Return-Path: X-Original-To: archive-asf-public-internal@cust-asf2.ponee.io Delivered-To: archive-asf-public-internal@cust-asf2.ponee.io Received: from cust-asf.ponee.io (cust-asf.ponee.io [163.172.22.183]) by cust-asf2.ponee.io (Postfix) with ESMTP id ED3A9200BCF for ; Mon, 5 Dec 2016 13:38:12 +0100 (CET) Received: by cust-asf.ponee.io (Postfix) id E91D0160AF9; Mon, 5 Dec 2016 12:38:12 +0000 (UTC) Delivered-To: archive-asf-public@cust-asf.ponee.io Received: from mail.apache.org (hermes.apache.org [140.211.11.3]) by cust-asf.ponee.io (Postfix) with SMTP id E0734160B38 for ; Mon, 5 Dec 2016 13:38:07 +0100 (CET) Received: (qmail 13164 invoked by uid 500); 5 Dec 2016 12:38:07 -0000 Mailing-List: contact commits-help@ignite.apache.org; run by ezmlm Precedence: bulk List-Help: List-Unsubscribe: List-Post: List-Id: Reply-To: dev@ignite.apache.org Delivered-To: mailing list commits@ignite.apache.org Received: (qmail 12201 invoked by uid 99); 5 Dec 2016 12:38:06 -0000 Received: from git1-us-west.apache.org (HELO git1-us-west.apache.org) (140.211.11.23) by apache.org (qpsmtpd/0.29) with ESMTP; Mon, 05 Dec 2016 12:38:06 +0000 Received: by git1-us-west.apache.org (ASF Mail Server at git1-us-west.apache.org, from userid 33) id EC40FF15AB; Mon, 5 Dec 2016 12:38:05 +0000 (UTC) Content-Type: text/plain; charset="us-ascii" MIME-Version: 1.0 Content-Transfer-Encoding: 7bit From: vozerov@apache.org To: commits@ignite.apache.org Date: Mon, 05 Dec 2016 12:38:12 -0000 Message-Id: <32e804ea1fa74b35b742b326e912e647@git.apache.org> In-Reply-To: <2c1cfa866e2c4f96838bc3183b3040f8@git.apache.org> References: <2c1cfa866e2c4f96838bc3183b3040f8@git.apache.org> X-Mailer: ASF-Git Admin Mailer Subject: [08/52] ignite git commit: IGNITE-2294: Implemented DML. archived-at: Mon, 05 Dec 2016 12:38:13 -0000 IGNITE-2294: Implemented DML. Project: http://git-wip-us.apache.org/repos/asf/ignite/repo Commit: http://git-wip-us.apache.org/repos/asf/ignite/commit/86d143bb Tree: http://git-wip-us.apache.org/repos/asf/ignite/tree/86d143bb Diff: http://git-wip-us.apache.org/repos/asf/ignite/diff/86d143bb Branch: refs/heads/master Commit: 86d143bb8bdbdccba9209762b1fd802d5dc40268 Parents: 93c3ccd Author: Alexander Paschenko Authored: Wed Nov 23 12:58:26 2016 +0300 Committer: devozerov Committed: Wed Nov 23 12:58:27 2016 +0300 ---------------------------------------------------------------------- .../clients/src/test/config/jdbc-bin-config.xml | 54 + .../jdbc2/JdbcAbstractDmlStatementSelfTest.java | 263 +++++ .../JdbcAbstractUpdateStatementSelfTest.java | 37 + ...BinaryMarshallerInsertStatementSelfTest.java | 37 + ...cBinaryMarshallerMergeStatementSelfTest.java | 37 + .../jdbc2/JdbcDeleteStatementSelfTest.java | 49 + .../jdbc2/JdbcInsertStatementSelfTest.java | 122 +++ .../jdbc2/JdbcMergeStatementSelfTest.java | 91 ++ .../jdbc/suite/IgniteJdbcDriverTestSuite.java | 5 + .../java/org/apache/ignite/IgniteCache.java | 2 + .../binary/BinaryAbstractIdentityResolver.java | 53 + .../binary/BinaryArrayIdentityResolver.java | 224 ++++ .../binary/BinaryFieldIdentityResolver.java | 307 ++++++ .../ignite/binary/BinaryIdentityResolver.java | 42 + .../ignite/binary/BinaryTypeConfiguration.java | 27 +- .../org/apache/ignite/cache/QueryEntity.java | 27 + .../ignite/cache/query/SqlFieldsQuery.java | 2 +- .../configuration/CacheConfiguration.java | 19 +- .../internal/binary/BinaryClassDescriptor.java | 15 + .../ignite/internal/binary/BinaryContext.java | 76 +- .../ignite/internal/binary/BinaryFieldImpl.java | 10 +- .../internal/binary/BinaryObjectExImpl.java | 90 +- .../internal/binary/BinaryObjectImpl.java | 48 +- .../binary/BinaryObjectOffheapImpl.java | 44 +- .../internal/binary/BinaryPrimitives.java | 24 + .../binary/BinarySerializedFieldComparator.java | 343 ++++++ .../ignite/internal/binary/BinaryUtils.java | 2 +- .../internal/binary/BinaryWriterExImpl.java | 47 +- .../binary/builder/BinaryObjectBuilderImpl.java | 6 + .../streams/BinaryAbstractInputStream.java | 5 + .../streams/BinaryAbstractOutputStream.java | 5 + .../binary/streams/BinaryHeapInputStream.java | 5 + .../binary/streams/BinaryHeapOutputStream.java | 5 + .../streams/BinaryOffheapInputStream.java | 10 + .../streams/BinaryOffheapOutputStream.java | 4 +- .../internal/binary/streams/BinaryStream.java | 12 +- .../ignite/internal/jdbc2/JdbcConnection.java | 17 + .../internal/jdbc2/JdbcPreparedStatement.java | 71 +- .../ignite/internal/jdbc2/JdbcQueryTask.java | 5 +- .../ignite/internal/jdbc2/JdbcQueryTaskV2.java | 406 +++++++ .../ignite/internal/jdbc2/JdbcResultSet.java | 27 + .../internal/jdbc2/JdbcSqlFieldsQuery.java | 49 + .../ignite/internal/jdbc2/JdbcStatement.java | 213 +++- .../apache/ignite/internal/jdbc2/JdbcUtils.java | 25 +- .../processors/cache/QueryCursorImpl.java | 31 +- .../cache/query/GridCacheTwoStepQuery.java | 253 ----- .../cache/query/IgniteQueryErrorCode.java | 91 ++ .../memory/PlatformInputStreamImpl.java | 10 + .../memory/PlatformOutputStreamImpl.java | 12 + .../processors/query/GridQueryIndexing.java | 23 +- .../processors/query/GridQueryProcessor.java | 578 ++++++++-- .../processors/query/GridQueryProperty.java | 20 + .../query/GridQueryTypeDescriptor.java | 25 + .../processors/query/IgniteSQLException.java | 89 ++ .../util/lang/IgniteSingletonIterator.java | 56 + .../BinaryArrayIdentityResolverSelfTest.java | 300 +++++ .../BinaryFieldIdentityResolverSelfTest.java | 333 ++++++ ...ryIdentityResolverConfigurationSelfTest.java | 138 +++ .../BinarySerialiedFieldComparatorSelfTest.java | 568 ++++++++++ .../GridCacheBinaryObjectsAbstractSelfTest.java | 260 ++++- .../IgniteBinaryObjectsTestSuite.java | 10 + .../cache/query/GridCacheTwoStepQuery.java | 253 +++++ .../query/h2/DmlStatementsProcessor.java | 1027 ++++++++++++++++++ .../query/h2/GridH2ResultSetIterator.java | 3 +- .../processors/query/h2/IgniteH2Indexing.java | 124 ++- .../query/h2/dml/FastUpdateArgument.java | 27 + .../query/h2/dml/FastUpdateArguments.java | 53 + .../query/h2/dml/KeyValueSupplier.java | 30 + .../processors/query/h2/dml/UpdateMode.java | 36 + .../processors/query/h2/dml/UpdatePlan.java | 121 +++ .../query/h2/dml/UpdatePlanBuilder.java | 502 +++++++++ .../processors/query/h2/dml/package-info.java | 22 + .../query/h2/opt/GridH2RowDescriptor.java | 23 + .../query/h2/opt/GridH2TreeIndex.java | 2 +- .../processors/query/h2/sql/DmlAstUtils.java | 599 ++++++++++ .../processors/query/h2/sql/GridSqlArray.java | 8 + .../processors/query/h2/sql/GridSqlConst.java | 6 + .../processors/query/h2/sql/GridSqlDelete.java | 68 ++ .../query/h2/sql/GridSqlFunction.java | 5 +- .../processors/query/h2/sql/GridSqlInsert.java | 149 +++ .../processors/query/h2/sql/GridSqlKeyword.java | 46 + .../processors/query/h2/sql/GridSqlMerge.java | 143 +++ .../processors/query/h2/sql/GridSqlQuery.java | 44 +- .../query/h2/sql/GridSqlQueryParser.java | 323 +++++- .../query/h2/sql/GridSqlQuerySplitter.java | 6 +- .../processors/query/h2/sql/GridSqlSelect.java | 3 +- .../query/h2/sql/GridSqlStatement.java | 64 ++ .../processors/query/h2/sql/GridSqlUpdate.java | 105 ++ .../h2/twostep/GridReduceQueryExecutor.java | 28 +- ...niteCacheAbstractInsertSqlQuerySelfTest.java | 567 ++++++++++ .../IgniteCacheAbstractSqlDmlQuerySelfTest.java | 219 ++++ .../IgniteCacheDeleteSqlQuerySelfTest.java | 81 ++ .../IgniteCacheInsertSqlQuerySelfTest.java | 203 ++++ .../cache/IgniteCacheMergeSqlQuerySelfTest.java | 153 +++ .../IgniteCacheUpdateSqlQuerySelfTest.java | 150 +++ .../IgniteCacheAtomicFieldsQuerySelfTest.java | 21 - .../h2/GridIndexingSpiAbstractSelfTest.java | 40 +- .../query/h2/sql/GridQueryParsingTest.java | 109 +- .../IgniteCacheQuerySelfTestSuite.java | 9 + .../config/benchmark-bin-identity.properties | 94 ++ .../config/benchmark-multicast.properties | 15 + .../config/benchmark-sql-dml.properties | 72 ++ modules/yardstick/config/ignite-base-config.xml | 73 +- .../config/ignite-bin-multicast-config.xml | 86 ++ .../cache/IgniteBinaryIdentityBenchmark.java | 108 ++ .../cache/IgniteBinaryIdentityGetBenchmark.java | 34 + .../cache/IgniteBinaryIdentityPutBenchmark.java | 35 + .../IgniteFieldsBinaryIdentityGetBenchmark.java | 30 + .../IgniteFieldsBinaryIdentityPutBenchmark.java | 30 + .../IgniteLegacyBinaryIdentityGetBenchmark.java | 30 + .../IgniteLegacyBinaryIdentityPutBenchmark.java | 30 + ...IgnitePutIfAbsentIndexedValue1Benchmark.java | 45 + .../IgniteReplaceIndexedValue1Benchmark.java | 79 ++ .../cache/dml/IgniteSqlDeleteBenchmark.java | 83 ++ .../dml/IgniteSqlDeleteFilteredBenchmark.java | 88 ++ .../IgniteSqlInsertIndexedValue1Benchmark.java | 48 + .../IgniteSqlInsertIndexedValue2Benchmark.java | 48 + .../IgniteSqlInsertIndexedValue8Benchmark.java | 48 + .../cache/dml/IgniteSqlMergeAllBenchmark.java | 82 ++ .../cache/dml/IgniteSqlMergeBenchmark.java | 42 + .../IgniteSqlMergeIndexedValue1Benchmark.java | 43 + .../IgniteSqlMergeIndexedValue2Benchmark.java | 43 + .../IgniteSqlMergeIndexedValue8Benchmark.java | 43 + .../cache/dml/IgniteSqlMergeQueryBenchmark.java | 116 ++ .../cache/dml/IgniteSqlUpdateBenchmark.java | 82 ++ .../dml/IgniteSqlUpdateFilteredBenchmark.java | 88 ++ .../yardstick/cache/model/SampleValue.java | 2 + 127 files changed, 12020 insertions(+), 728 deletions(-) ---------------------------------------------------------------------- http://git-wip-us.apache.org/repos/asf/ignite/blob/86d143bb/modules/clients/src/test/config/jdbc-bin-config.xml ---------------------------------------------------------------------- diff --git a/modules/clients/src/test/config/jdbc-bin-config.xml b/modules/clients/src/test/config/jdbc-bin-config.xml new file mode 100644 index 0000000..69f85a1 --- /dev/null +++ b/modules/clients/src/test/config/jdbc-bin-config.xml @@ -0,0 +1,54 @@ + + + + + + + + + + + + + + + + + + + + + + + 127.0.0.1:47500..47549 + + + + + + + + + + http://git-wip-us.apache.org/repos/asf/ignite/blob/86d143bb/modules/clients/src/test/java/org/apache/ignite/internal/jdbc2/JdbcAbstractDmlStatementSelfTest.java ---------------------------------------------------------------------- diff --git a/modules/clients/src/test/java/org/apache/ignite/internal/jdbc2/JdbcAbstractDmlStatementSelfTest.java b/modules/clients/src/test/java/org/apache/ignite/internal/jdbc2/JdbcAbstractDmlStatementSelfTest.java new file mode 100644 index 0000000..4a97aef --- /dev/null +++ b/modules/clients/src/test/java/org/apache/ignite/internal/jdbc2/JdbcAbstractDmlStatementSelfTest.java @@ -0,0 +1,263 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one or more + * contributor license agreements. See the NOTICE file distributed with + * this work for additional information regarding copyright ownership. + * The ASF licenses this file to You under the Apache License, Version 2.0 + * (the "License"); you may not use this file except in compliance with + * the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package org.apache.ignite.internal.jdbc2; + +import java.io.Serializable; +import java.sql.Connection; +import java.sql.DriverManager; +import java.sql.ResultSet; +import java.sql.Statement; +import java.util.Collections; +import org.apache.ignite.cache.CachePeekMode; +import org.apache.ignite.cache.QueryEntity; +import org.apache.ignite.cache.query.annotations.QuerySqlField; +import org.apache.ignite.configuration.CacheConfiguration; +import org.apache.ignite.configuration.ConnectorConfiguration; +import org.apache.ignite.configuration.IgniteConfiguration; +import org.apache.ignite.internal.binary.BinaryMarshaller; +import org.apache.ignite.internal.util.typedef.F; +import org.apache.ignite.spi.discovery.tcp.TcpDiscoverySpi; +import org.apache.ignite.spi.discovery.tcp.ipfinder.TcpDiscoveryIpFinder; +import org.apache.ignite.spi.discovery.tcp.ipfinder.vm.TcpDiscoveryVmIpFinder; +import org.apache.ignite.testframework.junits.common.GridCommonAbstractTest; + +import static org.apache.ignite.IgniteJdbcDriver.CFG_URL_PREFIX; +import static org.apache.ignite.cache.CacheMode.PARTITIONED; +import static org.apache.ignite.cache.CacheWriteSynchronizationMode.FULL_SYNC; + +/** + * Statement test. + */ +public abstract class JdbcAbstractDmlStatementSelfTest extends GridCommonAbstractTest { + /** IP finder. */ + private static final TcpDiscoveryIpFinder IP_FINDER = new TcpDiscoveryVmIpFinder(true); + + /** JDBC URL. */ + private static final String BASE_URL = CFG_URL_PREFIX + "modules/clients/src/test/config/jdbc-config.xml"; + + /** JDBC URL for tests involving binary objects manipulation. */ + static final String BASE_URL_BIN = CFG_URL_PREFIX + "modules/clients/src/test/config/jdbc-bin-config.xml"; + + /** SQL SELECT query for verification. */ + private static final String SQL_SELECT = "select _key, id, firstName, lastName, age from Person"; + + /** Connection. */ + protected Connection conn; + + /** {@inheritDoc} */ + @Override protected IgniteConfiguration getConfiguration(String gridName) throws Exception { + return getConfiguration0(gridName); + } + + /** + * @param gridName Grid name. + * @return Grid configuration used for starting the grid. + * @throws Exception If failed. + */ + private IgniteConfiguration getConfiguration0(String gridName) throws Exception { + IgniteConfiguration cfg = super.getConfiguration(gridName); + + CacheConfiguration cache = defaultCacheConfiguration(); + + cache.setCacheMode(PARTITIONED); + cache.setBackups(1); + cache.setWriteSynchronizationMode(FULL_SYNC); + cache.setIndexedTypes( + String.class, Person.class + ); + + cfg.setCacheConfiguration(cache); + + TcpDiscoverySpi disco = new TcpDiscoverySpi(); + + disco.setIpFinder(IP_FINDER); + + cfg.setDiscoverySpi(disco); + + cfg.setConnectorConfiguration(new ConnectorConfiguration()); + + return cfg; + } + + /** + * @param gridName Grid name. + * @return Grid configuration used for starting the grid ready for manipulating binary objects. + * @throws Exception If failed. + */ + IgniteConfiguration getBinaryConfiguration(String gridName) throws Exception { + IgniteConfiguration cfg = getConfiguration0(gridName); + + cfg.setMarshaller(new BinaryMarshaller()); + + CacheConfiguration ccfg = cfg.getCacheConfiguration()[0]; + + ccfg.getQueryEntities().clear(); + + QueryEntity e = new QueryEntity(); + + e.setKeyType(String.class.getName()); + e.setValueType("Person"); + + e.addQueryField("id", Integer.class.getName(), null); + e.addQueryField("age", Integer.class.getName(), null); + e.addQueryField("firstName", String.class.getName(), null); + e.addQueryField("lastName", String.class.getName(), null); + + ccfg.setQueryEntities(Collections.singletonList(e)); + + return cfg; + } + + /** {@inheritDoc} */ + @Override protected void beforeTestsStarted() throws Exception { + startGridsMultiThreaded(3); + + Class.forName("org.apache.ignite.IgniteJdbcDriver"); + } + + /** {@inheritDoc} */ + @Override protected void afterTestsStopped() throws Exception { + stopAllGrids(); + } + + /** {@inheritDoc} */ + @Override protected void beforeTest() throws Exception { + conn = DriverManager.getConnection(getCfgUrl()); + } + + /** + * @return URL of XML configuration file. + */ + protected String getCfgUrl() { + return BASE_URL; + } + + /** {@inheritDoc} */ + @Override protected void afterTest() throws Exception { + try (Statement selStmt = conn.createStatement()) { + assert selStmt.execute(SQL_SELECT); + + ResultSet rs = selStmt.getResultSet(); + + assert rs != null; + + while (rs.next()) { + int id = rs.getInt("id"); + + switch (id) { + case 1: + assertEquals("p1", rs.getString("_key")); + assertEquals("John", rs.getString("firstName")); + assertEquals("White", rs.getString("lastName")); + assertEquals(25, rs.getInt("age")); + break; + + case 2: + assertEquals("p2", rs.getString("_key")); + assertEquals("Joe", rs.getString("firstName")); + assertEquals("Black", rs.getString("lastName")); + assertEquals(35, rs.getInt("age")); + break; + + case 3: + assertEquals("p3", rs.getString("_key")); + assertEquals("Mike", rs.getString("firstName")); + assertEquals("Green", rs.getString("lastName")); + assertEquals(40, rs.getInt("age")); + break; + + case 4: + assertEquals("p4", rs.getString("_key")); + assertEquals("Leah", rs.getString("firstName")); + assertEquals("Grey", rs.getString("lastName")); + assertEquals(22, rs.getInt("age")); + break; + + default: + assert false : "Invalid ID: " + id; + } + } + } + + grid(0).cache(null).clear(); + + assertEquals(0, grid(0).cache(null).size(CachePeekMode.ALL)); + } + + /** + * Person. + */ + @SuppressWarnings("UnusedDeclaration") + static class Person implements Serializable { + /** ID. */ + @QuerySqlField + private final int id; + + /** First name. */ + @QuerySqlField + private final String firstName; + + /** Last name. */ + @QuerySqlField + private final String lastName; + + /** Age. */ + @QuerySqlField + private final int age; + + /** + * @param id ID. + * @param firstName First name. + * @param lastName Last name. + * @param age Age. + */ + Person(int id, String firstName, String lastName, int age) { + assert !F.isEmpty(firstName); + assert !F.isEmpty(lastName); + assert age > 0; + + this.id = id; + this.firstName = firstName; + this.lastName = lastName; + this.age = age; + } + + /** {@inheritDoc} */ + @Override public boolean equals(Object o) { + if (this == o) return true; + if (o == null || getClass() != o.getClass()) return false; + + Person person = (Person) o; + + if (id != person.id) return false; + if (age != person.age) return false; + if (firstName != null ? !firstName.equals(person.firstName) : person.firstName != null) return false; + return lastName != null ? lastName.equals(person.lastName) : person.lastName == null; + + } + + /** {@inheritDoc} */ + @Override public int hashCode() { + int result = id; + result = 31 * result + (firstName != null ? firstName.hashCode() : 0); + result = 31 * result + (lastName != null ? lastName.hashCode() : 0); + result = 31 * result + age; + return result; + } + } +} http://git-wip-us.apache.org/repos/asf/ignite/blob/86d143bb/modules/clients/src/test/java/org/apache/ignite/internal/jdbc2/JdbcAbstractUpdateStatementSelfTest.java ---------------------------------------------------------------------- diff --git a/modules/clients/src/test/java/org/apache/ignite/internal/jdbc2/JdbcAbstractUpdateStatementSelfTest.java b/modules/clients/src/test/java/org/apache/ignite/internal/jdbc2/JdbcAbstractUpdateStatementSelfTest.java new file mode 100644 index 0000000..a20b815 --- /dev/null +++ b/modules/clients/src/test/java/org/apache/ignite/internal/jdbc2/JdbcAbstractUpdateStatementSelfTest.java @@ -0,0 +1,37 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one or more + * contributor license agreements. See the NOTICE file distributed with + * this work for additional information regarding copyright ownership. + * The ASF licenses this file to You under the Apache License, Version 2.0 + * (the "License"); you may not use this file except in compliance with + * the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package org.apache.ignite.internal.jdbc2; + +import java.sql.Statement; + +public abstract class JdbcAbstractUpdateStatementSelfTest extends JdbcAbstractDmlStatementSelfTest { + /** SQL query to populate cache. */ + private static final String ITEMS_SQL = "insert into Person(_key, id, firstName, lastName, age) values " + + "('p1', 1, 'John', 'White', 25), " + + "('p2', 2, 'Joe', 'Black', 35), " + + "('p3', 3, 'Mike', 'Green', 40)"; + + /** {@inheritDoc} */ + @Override protected void beforeTest() throws Exception { + super.beforeTest(); + jcache(0).clear(); + try (Statement s = conn.createStatement()) { + s.executeUpdate(ITEMS_SQL); + } + } +} http://git-wip-us.apache.org/repos/asf/ignite/blob/86d143bb/modules/clients/src/test/java/org/apache/ignite/internal/jdbc2/JdbcBinaryMarshallerInsertStatementSelfTest.java ---------------------------------------------------------------------- diff --git a/modules/clients/src/test/java/org/apache/ignite/internal/jdbc2/JdbcBinaryMarshallerInsertStatementSelfTest.java b/modules/clients/src/test/java/org/apache/ignite/internal/jdbc2/JdbcBinaryMarshallerInsertStatementSelfTest.java new file mode 100644 index 0000000..667e9f0 --- /dev/null +++ b/modules/clients/src/test/java/org/apache/ignite/internal/jdbc2/JdbcBinaryMarshallerInsertStatementSelfTest.java @@ -0,0 +1,37 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one or more + * contributor license agreements. See the NOTICE file distributed with + * this work for additional information regarding copyright ownership. + * The ASF licenses this file to You under the Apache License, Version 2.0 + * (the "License"); you may not use this file except in compliance with + * the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package org.apache.ignite.internal.jdbc2; + +import org.apache.ignite.configuration.IgniteConfiguration; +import org.apache.ignite.internal.binary.BinaryMarshaller; +import org.apache.ignite.testframework.config.GridTestProperties; + +/** + * JDBC test of INSERT statement w/binary marshaller - no nodes know about classes. + */ +public class JdbcBinaryMarshallerInsertStatementSelfTest extends JdbcInsertStatementSelfTest { + /** {@inheritDoc} */ + @Override protected String getCfgUrl() { + return BASE_URL_BIN; + } + + /** {@inheritDoc} */ + @Override protected IgniteConfiguration getConfiguration(String gridName) throws Exception { + return getBinaryConfiguration(gridName); + } +} http://git-wip-us.apache.org/repos/asf/ignite/blob/86d143bb/modules/clients/src/test/java/org/apache/ignite/internal/jdbc2/JdbcBinaryMarshallerMergeStatementSelfTest.java ---------------------------------------------------------------------- diff --git a/modules/clients/src/test/java/org/apache/ignite/internal/jdbc2/JdbcBinaryMarshallerMergeStatementSelfTest.java b/modules/clients/src/test/java/org/apache/ignite/internal/jdbc2/JdbcBinaryMarshallerMergeStatementSelfTest.java new file mode 100644 index 0000000..93451e7 --- /dev/null +++ b/modules/clients/src/test/java/org/apache/ignite/internal/jdbc2/JdbcBinaryMarshallerMergeStatementSelfTest.java @@ -0,0 +1,37 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one or more + * contributor license agreements. See the NOTICE file distributed with + * this work for additional information regarding copyright ownership. + * The ASF licenses this file to You under the Apache License, Version 2.0 + * (the "License"); you may not use this file except in compliance with + * the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package org.apache.ignite.internal.jdbc2; + +import org.apache.ignite.configuration.IgniteConfiguration; +import org.apache.ignite.internal.binary.BinaryMarshaller; +import org.apache.ignite.testframework.config.GridTestProperties; + +/** + * JDBC test of MERGE statement w/binary marshaller - no nodes know about classes. + */ +public class JdbcBinaryMarshallerMergeStatementSelfTest extends JdbcMergeStatementSelfTest { + /** {@inheritDoc} */ + @Override protected String getCfgUrl() { + return BASE_URL_BIN; + } + + /** {@inheritDoc} */ + @Override protected IgniteConfiguration getConfiguration(String gridName) throws Exception { + return getBinaryConfiguration(gridName); + } +} http://git-wip-us.apache.org/repos/asf/ignite/blob/86d143bb/modules/clients/src/test/java/org/apache/ignite/internal/jdbc2/JdbcDeleteStatementSelfTest.java ---------------------------------------------------------------------- diff --git a/modules/clients/src/test/java/org/apache/ignite/internal/jdbc2/JdbcDeleteStatementSelfTest.java b/modules/clients/src/test/java/org/apache/ignite/internal/jdbc2/JdbcDeleteStatementSelfTest.java new file mode 100644 index 0000000..d55c979 --- /dev/null +++ b/modules/clients/src/test/java/org/apache/ignite/internal/jdbc2/JdbcDeleteStatementSelfTest.java @@ -0,0 +1,49 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one or more + * contributor license agreements. See the NOTICE file distributed with + * this work for additional information regarding copyright ownership. + * The ASF licenses this file to You under the Apache License, Version 2.0 + * (the "License"); you may not use this file except in compliance with + * the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package org.apache.ignite.internal.jdbc2; + +import java.sql.SQLException; +import java.util.Arrays; +import java.util.HashSet; + +/** + * + */ +public class JdbcDeleteStatementSelfTest extends JdbcAbstractUpdateStatementSelfTest { + /** + * + */ + public void testExecute() throws SQLException { + conn.createStatement().execute("delete from Person where cast(substring(_key, 2, 1) as int) % 2 = 0"); + + assertFalse(jcache(0).containsKey("p2")); + assertTrue(jcache(0).containsKeys(new HashSet(Arrays.asList("p1", "p3")))); + } + + /** + * + */ + public void testExecuteUpdate() throws SQLException { + int res = + conn.createStatement().executeUpdate("delete from Person where cast(substring(_key, 2, 1) as int) % 2 = 0"); + + assertEquals(1, res); + assertFalse(jcache(0).containsKey("p2")); + assertTrue(jcache(0).containsKeys(new HashSet(Arrays.asList("p1", "p3")))); + } +} http://git-wip-us.apache.org/repos/asf/ignite/blob/86d143bb/modules/clients/src/test/java/org/apache/ignite/internal/jdbc2/JdbcInsertStatementSelfTest.java ---------------------------------------------------------------------- diff --git a/modules/clients/src/test/java/org/apache/ignite/internal/jdbc2/JdbcInsertStatementSelfTest.java b/modules/clients/src/test/java/org/apache/ignite/internal/jdbc2/JdbcInsertStatementSelfTest.java new file mode 100644 index 0000000..7fc92de --- /dev/null +++ b/modules/clients/src/test/java/org/apache/ignite/internal/jdbc2/JdbcInsertStatementSelfTest.java @@ -0,0 +1,122 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one or more + * contributor license agreements. See the NOTICE file distributed with + * this work for additional information regarding copyright ownership. + * The ASF licenses this file to You under the Apache License, Version 2.0 + * (the "License"); you may not use this file except in compliance with + * the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package org.apache.ignite.internal.jdbc2; + +import java.sql.PreparedStatement; +import java.sql.SQLException; +import java.sql.Statement; +import java.util.Arrays; +import java.util.HashSet; +import java.util.concurrent.Callable; +import org.apache.ignite.IgniteException; +import org.apache.ignite.testframework.GridTestUtils; + +/** + * Statement test. + */ +public class JdbcInsertStatementSelfTest extends JdbcAbstractDmlStatementSelfTest { + /** SQL query. */ + private static final String SQL = "insert into Person(_key, id, firstName, lastName, age) values " + + "('p1', 1, 'John', 'White', 25), " + + "('p2', 2, 'Joe', 'Black', 35), " + + "('p3', 3, 'Mike', 'Green', 40)"; + + /** SQL query. */ + private static final String SQL_PREPARED = "insert into Person(_key, id, firstName, lastName, age) values " + + "(?, ?, ?, ?, ?), (?, ?, ?, ?, ?)"; + + /** Statement. */ + private Statement stmt; + + /** Prepared statement. */ + private PreparedStatement prepStmt; + + /** {@inheritDoc} */ + @Override protected void beforeTest() throws Exception { + super.beforeTest(); + stmt = conn.createStatement(); + prepStmt = conn.prepareStatement(SQL_PREPARED); + + assertNotNull(stmt); + assertFalse(stmt.isClosed()); + + assertNotNull(prepStmt); + assertFalse(prepStmt.isClosed()); + } + + /** {@inheritDoc} */ + @Override protected void afterTest() throws Exception { + super.afterTest(); + + if (stmt != null && !stmt.isClosed()) + stmt.close(); + + if (prepStmt != null && !prepStmt.isClosed()) + prepStmt.close(); + + conn.close(); + + assertTrue(prepStmt.isClosed()); + assertTrue(stmt.isClosed()); + assertTrue(conn.isClosed()); + } + + /** + * @throws SQLException If failed. + */ + public void testExecuteUpdate() throws SQLException { + int res = stmt.executeUpdate(SQL); + + assertEquals(3, res); + } + + /** + * @throws SQLException If failed. + */ + public void testExecute() throws SQLException { + boolean res = stmt.execute(SQL); + + assertEquals(false, res); + } + + /** + * + */ + public void testDuplicateKeys() { + jcache(0).put("p2", new Person(2, "Joe", "Black", 35)); + + Throwable reason = GridTestUtils.assertThrows(log, new Callable() { + /** {@inheritDoc} */ + @Override public Object call() throws Exception { + return stmt.execute(SQL); + } + }, SQLException.class, null); + + assertNotNull(reason.getCause()); + + reason = reason.getCause().getCause(); + + assertNotNull(reason); + + assertEquals(IgniteException.class, reason.getClass()); + + assertEquals("Failed to INSERT some keys because they are already in cache [keys=[p2]]", reason.getMessage()); + + assertEquals(3, jcache(0).withKeepBinary().getAll(new HashSet<>(Arrays.asList("p1", "p2", "p3"))).size()); + } +} http://git-wip-us.apache.org/repos/asf/ignite/blob/86d143bb/modules/clients/src/test/java/org/apache/ignite/internal/jdbc2/JdbcMergeStatementSelfTest.java ---------------------------------------------------------------------- diff --git a/modules/clients/src/test/java/org/apache/ignite/internal/jdbc2/JdbcMergeStatementSelfTest.java b/modules/clients/src/test/java/org/apache/ignite/internal/jdbc2/JdbcMergeStatementSelfTest.java new file mode 100644 index 0000000..ecf6032 --- /dev/null +++ b/modules/clients/src/test/java/org/apache/ignite/internal/jdbc2/JdbcMergeStatementSelfTest.java @@ -0,0 +1,91 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one or more + * contributor license agreements. See the NOTICE file distributed with + * this work for additional information regarding copyright ownership. + * The ASF licenses this file to You under the Apache License, Version 2.0 + * (the "License"); you may not use this file except in compliance with + * the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package org.apache.ignite.internal.jdbc2; + +import java.sql.PreparedStatement; +import java.sql.SQLException; +import java.sql.Statement; + +/** + * MERGE statement test. + */ +public class JdbcMergeStatementSelfTest extends JdbcAbstractDmlStatementSelfTest { + /** SQL query. */ + private static final String SQL = "merge into Person(_key, id, firstName, lastName, age) values " + + "('p1', 1, 'John', 'White', 25), " + + "('p2', 2, 'Joe', 'Black', 35), " + + "('p3', 3, 'Mike', 'Green', 40)"; + + /** SQL query. */ + protected static final String SQL_PREPARED = "merge into Person(_key, id, firstName, lastName, age) values " + + "(?, ?, ?, ?, ?), (?, ?, ?, ?, ?)"; + + /** Statement. */ + protected Statement stmt; + + /** Prepared statement. */ + protected PreparedStatement prepStmt; + + /** {@inheritDoc} */ + @Override protected void beforeTest() throws Exception { + super.beforeTest(); + stmt = conn.createStatement(); + prepStmt = conn.prepareStatement(SQL_PREPARED); + + assertNotNull(stmt); + assertFalse(stmt.isClosed()); + + assertNotNull(prepStmt); + assertFalse(prepStmt.isClosed()); + } + + /** {@inheritDoc} */ + @Override protected void afterTest() throws Exception { + super.afterTest(); + + if (stmt != null && !stmt.isClosed()) + stmt.close(); + + if (prepStmt != null && !prepStmt.isClosed()) + prepStmt.close(); + + conn.close(); + + assertTrue(prepStmt.isClosed()); + assertTrue(stmt.isClosed()); + assertTrue(conn.isClosed()); + } + + /** + * @throws SQLException If failed. + */ + public void testExecuteUpdate() throws SQLException { + int res = stmt.executeUpdate(SQL); + + assertEquals(3, res); + } + + /** + * @throws SQLException If failed. + */ + public void testExecute() throws SQLException { + boolean res = stmt.execute(SQL); + + assertEquals(false, res); + } +} http://git-wip-us.apache.org/repos/asf/ignite/blob/86d143bb/modules/clients/src/test/java/org/apache/ignite/jdbc/suite/IgniteJdbcDriverTestSuite.java ---------------------------------------------------------------------- diff --git a/modules/clients/src/test/java/org/apache/ignite/jdbc/suite/IgniteJdbcDriverTestSuite.java b/modules/clients/src/test/java/org/apache/ignite/jdbc/suite/IgniteJdbcDriverTestSuite.java index b1053b0..048643b 100644 --- a/modules/clients/src/test/java/org/apache/ignite/jdbc/suite/IgniteJdbcDriverTestSuite.java +++ b/modules/clients/src/test/java/org/apache/ignite/jdbc/suite/IgniteJdbcDriverTestSuite.java @@ -62,6 +62,11 @@ public class IgniteJdbcDriverTestSuite extends TestSuite { suite.addTest(new TestSuite(org.apache.ignite.internal.jdbc2.JdbcEmptyCacheSelfTest.class)); suite.addTest(new TestSuite(org.apache.ignite.internal.jdbc2.JdbcLocalCachesSelfTest.class)); suite.addTest(new TestSuite(org.apache.ignite.internal.jdbc2.JdbcNoDefaultCacheTest.class)); + suite.addTest(new TestSuite(org.apache.ignite.internal.jdbc2.JdbcMergeStatementSelfTest.class)); + suite.addTest(new TestSuite(org.apache.ignite.internal.jdbc2.JdbcBinaryMarshallerMergeStatementSelfTest.class)); + suite.addTest(new TestSuite(org.apache.ignite.internal.jdbc2.JdbcInsertStatementSelfTest.class)); + suite.addTest(new TestSuite(org.apache.ignite.internal.jdbc2.JdbcBinaryMarshallerInsertStatementSelfTest.class)); + suite.addTest(new TestSuite(org.apache.ignite.internal.jdbc2.JdbcDeleteStatementSelfTest.class)); return suite; } http://git-wip-us.apache.org/repos/asf/ignite/blob/86d143bb/modules/core/src/main/java/org/apache/ignite/IgniteCache.java ---------------------------------------------------------------------- diff --git a/modules/core/src/main/java/org/apache/ignite/IgniteCache.java b/modules/core/src/main/java/org/apache/ignite/IgniteCache.java index 7eb6e91..d7bccf5 100644 --- a/modules/core/src/main/java/org/apache/ignite/IgniteCache.java +++ b/modules/core/src/main/java/org/apache/ignite/IgniteCache.java @@ -47,6 +47,7 @@ import org.apache.ignite.cache.query.QueryDetailMetrics; import org.apache.ignite.cache.query.QueryMetrics; import org.apache.ignite.cache.query.ScanQuery; import org.apache.ignite.cache.query.SpiQuery; +import org.apache.ignite.cache.query.SqlFieldsQuery; import org.apache.ignite.cache.query.SqlQuery; import org.apache.ignite.cache.query.TextQuery; import org.apache.ignite.cache.store.CacheStore; @@ -291,6 +292,7 @@ public interface IgniteCache extends javax.cache.Cache, IgniteAsyncS * @return Cursor. * @see ScanQuery * @see SqlQuery + * @see SqlFieldsQuery * @see TextQuery * @see SpiQuery */ http://git-wip-us.apache.org/repos/asf/ignite/blob/86d143bb/modules/core/src/main/java/org/apache/ignite/binary/BinaryAbstractIdentityResolver.java ---------------------------------------------------------------------- diff --git a/modules/core/src/main/java/org/apache/ignite/binary/BinaryAbstractIdentityResolver.java b/modules/core/src/main/java/org/apache/ignite/binary/BinaryAbstractIdentityResolver.java new file mode 100644 index 0000000..b3036e2 --- /dev/null +++ b/modules/core/src/main/java/org/apache/ignite/binary/BinaryAbstractIdentityResolver.java @@ -0,0 +1,53 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one or more + * contributor license agreements. See the NOTICE file distributed with + * this work for additional information regarding copyright ownership. + * The ASF licenses this file to You under the Apache License, Version 2.0 + * (the "License"); you may not use this file except in compliance with + * the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package org.apache.ignite.binary; + +/** + * Abstract identity resolver with common routines. + */ +public abstract class BinaryAbstractIdentityResolver implements BinaryIdentityResolver { + /** {@inheritDoc} */ + @Override public int hashCode(BinaryObject obj) { + if (obj == null) + throw new BinaryObjectException("Cannot calculate hash code because binary object is null."); + + return hashCode0(obj); + } + + /** {@inheritDoc} */ + @Override public boolean equals(BinaryObject o1, BinaryObject o2) { + return o1 == o2 || (o1 != null && o2 != null && equals0(o1, o2)); + } + + /** + * Internal hash code routine. + * + * @param obj Object. + * @return Result. + */ + protected abstract int hashCode0(BinaryObject obj); + + /** + * Internal equals routine. + * + * @param o1 First object. + * @param o2 Second object. + * @return Result. + */ + protected abstract boolean equals0(BinaryObject o1, BinaryObject o2); +} http://git-wip-us.apache.org/repos/asf/ignite/blob/86d143bb/modules/core/src/main/java/org/apache/ignite/binary/BinaryArrayIdentityResolver.java ---------------------------------------------------------------------- diff --git a/modules/core/src/main/java/org/apache/ignite/binary/BinaryArrayIdentityResolver.java b/modules/core/src/main/java/org/apache/ignite/binary/BinaryArrayIdentityResolver.java new file mode 100644 index 0000000..2f04c02 --- /dev/null +++ b/modules/core/src/main/java/org/apache/ignite/binary/BinaryArrayIdentityResolver.java @@ -0,0 +1,224 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one or more + * contributor license agreements. See the NOTICE file distributed with + * this work for additional information regarding copyright ownership. + * The ASF licenses this file to You under the Apache License, Version 2.0 + * (the "License"); you may not use this file except in compliance with + * the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package org.apache.ignite.binary; + +import java.util.Arrays; + +import org.apache.ignite.internal.binary.BinaryEnumObjectImpl; +import org.apache.ignite.internal.binary.BinaryObjectEx; +import org.apache.ignite.internal.binary.BinaryObjectExImpl; +import org.apache.ignite.internal.binary.BinaryPrimitives; +import org.apache.ignite.internal.util.typedef.internal.S; + +/** + * Identity resolver implementation which compares raw array content of the binary object. + *

+ * Hash code is calculated in the same way as {@link Arrays#hashCode(byte[])} does. + */ +public class BinaryArrayIdentityResolver extends BinaryAbstractIdentityResolver { + /** Singleton instance */ + private static final BinaryArrayIdentityResolver INSTANCE = new BinaryArrayIdentityResolver(); + + /** + * Get singleton instance. + * + * @return Singleton instance. + */ + public static BinaryArrayIdentityResolver instance() { + return INSTANCE; + } + + /** + * Default constructor. + */ + public BinaryArrayIdentityResolver() { + // No-op. + } + + /** {@inheritDoc} */ + @Override protected int hashCode0(BinaryObject obj) { + int hash = 1; + + if (obj instanceof BinaryObjectExImpl) { + BinaryObjectExImpl ex = (BinaryObjectExImpl)obj; + + int start = ex.dataStartOffset(); + int end = ex.footerStartOffset(); + + if (ex.hasArray()) { + // Handle heap object. + byte[] data = ex.array(); + + for (int i = start; i < end; i++) + hash = 31 * hash + data[i]; + } + else { + // Handle offheap object. + long ptr = ex.offheapAddress(); + + for (int i = start; i < end; i++) + hash = 31 * hash + BinaryPrimitives.readByte(ptr, i); + } + } + else if (obj instanceof BinaryEnumObjectImpl) { + int ord = obj.enumOrdinal(); + + // Construct hash as if it was an int serialized in little-endian form. + hash = 31 * hash + (ord & 0x000000FF); + hash = 31 * hash + (ord & 0x0000FF00); + hash = 31 * hash + (ord & 0x00FF0000); + hash = 31 * hash + (ord & 0xFF000000); + } + else + throw new BinaryObjectException("Array identity resolver cannot be used with provided BinaryObject " + + "implementation: " + obj.getClass().getName()); + + return hash; + } + + /** {@inheritDoc} */ + @Override protected boolean equals0(BinaryObject o1, BinaryObject o2) { + if (o1 instanceof BinaryObjectEx && o2 instanceof BinaryObjectEx) { + BinaryObjectEx ex1 = (BinaryObjectEx)o1; + BinaryObjectEx ex2 = (BinaryObjectEx)o2; + + if (ex1.typeId() != ex2.typeId()) + return false; + + if (ex1 instanceof BinaryObjectExImpl) { + // Handle regular object. + assert ex2 instanceof BinaryObjectExImpl; + + BinaryObjectExImpl exx1 = (BinaryObjectExImpl)ex1; + BinaryObjectExImpl exx2 = (BinaryObjectExImpl)ex2; + + if (exx1.hasArray()) + return exx2.hasArray() ? equalsHeap(exx1, exx2) : equalsHeapOffheap(exx1, exx2); + else + return exx2.hasArray() ? equalsHeapOffheap(exx2, exx1) : equalsOffheap(exx1, exx2); + } + else { + // Handle enums. + assert ex1 instanceof BinaryEnumObjectImpl; + assert ex2 instanceof BinaryEnumObjectImpl; + + return ex1.enumOrdinal() == ex2.enumOrdinal(); + } + } + + BinaryObject o = o1 instanceof BinaryObjectEx ? o2 : o1; + + throw new BinaryObjectException("Array identity resolver cannot be used with provided BinaryObject " + + "implementation: " + o.getClass().getName()); + } + + /** + * Compare two heap objects. + * + * @param o1 Object 1. + * @param o2 Object 2. + * @return Result. + */ + private static boolean equalsHeap(BinaryObjectExImpl o1, BinaryObjectExImpl o2) { + byte[] arr1 = o1.array(); + byte[] arr2 = o2.array(); + + assert arr1 != null && arr2 != null; + + int i = o1.dataStartOffset(); + int j = o2.dataStartOffset(); + + int end = o1.footerStartOffset(); + + // Check length. + if (end - i != o2.footerStartOffset() - j) + return false; + + for (; i < end; i++, j++) { + if (arr1[i] != arr2[j]) + return false; + } + + return true; + } + + /** + * Compare heap and offheap objects. + * + * @param o1 Object 1 (heap). + * @param o2 Object 2 (offheap). + * @return Result. + */ + private static boolean equalsHeapOffheap(BinaryObjectExImpl o1, BinaryObjectExImpl o2) { + byte[] arr1 = o1.array(); + long ptr2 = o2.offheapAddress(); + + assert arr1 != null && ptr2 != 0; + + int i = o1.dataStartOffset(); + int j = o2.dataStartOffset(); + + int end = o1.footerStartOffset(); + + // Check length. + if (end - i != o2.footerStartOffset() - j) + return false; + + for (; i < end; i++, j++) { + if (arr1[i] != BinaryPrimitives.readByte(ptr2, j)) + return false; + } + + return true; + } + + /** + * Compare two offheap objects. + * + * @param o1 Object 1. + * @param o2 Object 2. + * @return Result. + */ + private static boolean equalsOffheap(BinaryObjectExImpl o1, BinaryObjectExImpl o2) { + long ptr1 = o1.offheapAddress(); + long ptr2 = o2.offheapAddress(); + + assert ptr1 != 0 && ptr2 != 0; + + int i = o1.dataStartOffset(); + int j = o2.dataStartOffset(); + + int end = o1.footerStartOffset(); + + // Check length. + if (end - i != o2.footerStartOffset() - j) + return false; + + for (; i < end; i++, j++) { + if (BinaryPrimitives.readByte(ptr1, i) != BinaryPrimitives.readByte(ptr2, j)) + return false; + } + + return true; + } + + /** {@inheritDoc} */ + @Override public String toString() { + return S.toString(BinaryArrayIdentityResolver.class, this); + } +} http://git-wip-us.apache.org/repos/asf/ignite/blob/86d143bb/modules/core/src/main/java/org/apache/ignite/binary/BinaryFieldIdentityResolver.java ---------------------------------------------------------------------- diff --git a/modules/core/src/main/java/org/apache/ignite/binary/BinaryFieldIdentityResolver.java b/modules/core/src/main/java/org/apache/ignite/binary/BinaryFieldIdentityResolver.java new file mode 100644 index 0000000..c4fc869 --- /dev/null +++ b/modules/core/src/main/java/org/apache/ignite/binary/BinaryFieldIdentityResolver.java @@ -0,0 +1,307 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one or more + * contributor license agreements. See the NOTICE file distributed with + * this work for additional information regarding copyright ownership. + * The ASF licenses this file to You under the Apache License, Version 2.0 + * (the "License"); you may not use this file except in compliance with + * the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package org.apache.ignite.binary; + +import org.apache.ignite.internal.binary.BinaryEnumObjectImpl; +import org.apache.ignite.internal.binary.BinaryFieldImpl; +import org.apache.ignite.internal.binary.BinaryObjectExImpl; +import org.apache.ignite.internal.binary.BinarySerializedFieldComparator; +import org.apache.ignite.internal.util.typedef.F; +import org.apache.ignite.internal.util.typedef.internal.S; + +import java.util.HashMap; + +/** + * Identity resolver implementation which use the list of provided fields to calculate the hash code and to perform + * equality checks. + *

+ * Standard polynomial function with multiplier {@code 31} is used to calculate hash code. For example, for three + * fields {@code [a, b, c]}it would be {@code hash = 31 * (31 * a + b) + c}. Order of fields is important. + */ +public class BinaryFieldIdentityResolver extends BinaryAbstractIdentityResolver { + /** Mutex for synchronization. */ + private final Object mux = new Object(); + + /** Cached single accessor. */ + private volatile FieldAccessor accessor; + + /** Cached accessors used when multiple (typeId, schemaId) pairs are met. */ + private volatile HashMap accessors; + + /** Field names. */ + private String[] fieldNames; + + /** + * Default constructor. + */ + public BinaryFieldIdentityResolver() { + // No-op. + } + + /** + * Copy constructor. + * + * @param other Other instance. + */ + public BinaryFieldIdentityResolver(BinaryFieldIdentityResolver other) { + fieldNames = other.fieldNames; + } + + /** + * @return Fields list to hash/compare objects based upon. + */ + public String[] getFieldNames() { + return fieldNames; + } + + /** + * Set field names. + * + * @param fieldNames Field names. + * @return {@code this} for chaining. + */ + public BinaryFieldIdentityResolver setFieldNames(String... fieldNames) { + this.fieldNames = fieldNames; + + return this; + } + + /** {@inheritDoc} */ + @Override public int hashCode0(BinaryObject obj) { + if (obj instanceof BinaryObjectExImpl) { + BinaryObjectExImpl obj0 = (BinaryObjectExImpl)obj; + + if (obj0.hasSchema()) { + // Handle optimized case. + FieldAccessor accessor = accessor(obj0, obj0.typeId(), obj0.schemaId()); + + assert accessor != null; + + return accessor.hashCode(obj0); + } + } + else if (obj instanceof BinaryEnumObjectImpl) + throw new BinaryObjectException("Field identity resolver cannot be used with enums: " + obj); + + // Handle regular case. + int hash = 0; + + for (String fieldName : fieldNames) { + Object val = obj.field(fieldName); + + hash = 31 * hash + (val != null ? val.hashCode() : 0); + } + + return hash; + } + + /** {@inheritDoc} */ + @Override public boolean equals0(BinaryObject o1, BinaryObject o2) { + if (o1 instanceof BinaryObjectExImpl && o2 instanceof BinaryObjectExImpl) { + BinaryObjectExImpl ex1 = (BinaryObjectExImpl) o1; + BinaryObjectExImpl ex2 = (BinaryObjectExImpl) o2; + + int typeId = ex1.typeId(); + + if (typeId != ex2.typeId()) + return false; + + if (ex1.hasSchema() && ex2.hasSchema()) { + // Optimistic case: both objects have schemas. + int schemaId1 = ex1.schemaId(); + int schemaId2 = ex2.schemaId(); + + FieldAccessor accessor1 = accessor(ex1, typeId, schemaId1); + + FieldAccessor accessor2; + + if (schemaId1 == schemaId2) + accessor2 = accessor1; + else + accessor2 = accessor(ex2, typeId, schemaId2); + + // Even better case: compare fields without deserialization. + BinarySerializedFieldComparator comp1 = ex1.createFieldComparator(); + BinarySerializedFieldComparator comp2 = ex2.createFieldComparator(); + + for (int i = 0; i < fieldNames.length; i++) { + comp1.findField(accessor1.orders[i]); + comp2.findField(accessor2.orders[i]); + + if (!BinarySerializedFieldComparator.equals(comp1, comp2)) + return false; + } + + return true; + } + else + // Pessimistic case: object of unknown types, or without schemas. Have to read fields in usual way. + return equalsSlow(ex1, ex2); + } + + if (o1 instanceof BinaryEnumObjectImpl) + throw new BinaryObjectException("Field identity resolver cannot be used with enums: " + o1); + + if (o2 instanceof BinaryEnumObjectImpl) + throw new BinaryObjectException("Field identity resolver cannot be used with enums: " + o2); + + return o1.type().typeId() == o2.type().typeId() && equalsSlow(o1, o2); + } + + /** + * Slow-path equals routine: regular fields comparison. + * + * @param o1 Object 1. + * @param o2 Object 2. + * @return Result. + */ + private boolean equalsSlow(BinaryObject o1, BinaryObject o2) { + for (String fieldName : fieldNames) { + Object val1 = o1.field(fieldName); + Object val2 = o2.field(fieldName); + + if (!F.eq(val1, val2)) + return false; + } + + return true; + } + + /** + * Get fields accessor for the given object. + * + * @param obj Object. + * @param typId Type ID. + * @param schemaId Schema ID. + * @return Accessor. + */ + private FieldAccessor accessor(BinaryObjectExImpl obj, int typId, int schemaId) { + // Try getting single accessor. + FieldAccessor res = accessor; + + if (res != null && res.applicableTo(typId, schemaId)) + return res; + + // Try reading from map. + long key = ((long)typId << 32) + schemaId; + + HashMap accessors0 = accessors; + + if (accessors0 != null) { + res = accessors0.get(key); + + if (res != null) + return res; + } + + // Failed to get from cache, go to locking. + synchronized (mux) { + // Create accessor. + int[] orders = new int[fieldNames.length]; + + BinaryType type = obj.type(); + + for (int i = 0; i < fieldNames.length; i++) { + BinaryFieldImpl field = (BinaryFieldImpl)type.field(fieldNames[i]); + + orders[i] = field.fieldOrder(obj); + } + + res = new FieldAccessor(typId, schemaId, orders); + + // Set accessor. + if (accessor == null) + accessor = res; + else { + if (accessors == null) { + accessor = null; + + accessors0 = new HashMap<>(); + } + else + accessors0 = new HashMap<>(accessors); + + accessors0.put(key, res); + + accessors = accessors0; + } + } + + return res; + } + + /** {@inheritDoc} */ + @Override public String toString() { + return S.toString(BinaryFieldIdentityResolver.class, this); + } + + /** + * Optimized fields accessor. + */ + private static class FieldAccessor { + /** Type ID. */ + private final int typeId; + + /** Schema ID. */ + private final int schemaId; + + /** Field orders. */ + private final int[] orders; + + /** + * Constructor. + * + * @param typeId Type ID. + * @param schemaId Schema ID. + * @param orders Field orders. + */ + private FieldAccessor(int typeId, int schemaId, int[] orders) { + this.typeId = typeId; + this.schemaId = schemaId; + this.orders = orders; + } + + /** + * Check whether object is applicable to that hash code accessor. + * @param expTypeId Expected schema ID. + * @param expSchemaId Expected schema ID. + * @return {@code True} if matches. + */ + private boolean applicableTo(int expTypeId, int expSchemaId) { + return typeId == expTypeId && schemaId == expSchemaId; + } + + /** + * Calculate object hash code. + * + * @param obj Object. + * @return Hash code. + */ + private int hashCode(BinaryObjectExImpl obj) { + int hash = 0; + + for (int order : orders) { + Object val = obj.fieldByOrder(order); + + hash = 31 * hash + (val != null ? val.hashCode() : 0); + } + + return hash; + } + } +} http://git-wip-us.apache.org/repos/asf/ignite/blob/86d143bb/modules/core/src/main/java/org/apache/ignite/binary/BinaryIdentityResolver.java ---------------------------------------------------------------------- diff --git a/modules/core/src/main/java/org/apache/ignite/binary/BinaryIdentityResolver.java b/modules/core/src/main/java/org/apache/ignite/binary/BinaryIdentityResolver.java new file mode 100644 index 0000000..9796eca --- /dev/null +++ b/modules/core/src/main/java/org/apache/ignite/binary/BinaryIdentityResolver.java @@ -0,0 +1,42 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one or more + * contributor license agreements. See the NOTICE file distributed with + * this work for additional information regarding copyright ownership. + * The ASF licenses this file to You under the Apache License, Version 2.0 + * (the "License"); you may not use this file except in compliance with + * the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package org.apache.ignite.binary; + +import org.jetbrains.annotations.Nullable; + +/** + * Interface to compute hash codes for new binary objects and compare them for equality. + */ +public interface BinaryIdentityResolver { + /** + * Compute hash code for binary object. + * + * @param obj Binary object. + * @return Hash code value. + */ + public int hashCode(BinaryObject obj); + + /** + * Compare two binary objects for equality. + * + * @param o1 First object. + * @param o2 Second object. + * @return {@code True} if both objects are equal. + */ + public boolean equals(@Nullable BinaryObject o1, @Nullable BinaryObject o2); +} http://git-wip-us.apache.org/repos/asf/ignite/blob/86d143bb/modules/core/src/main/java/org/apache/ignite/binary/BinaryTypeConfiguration.java ---------------------------------------------------------------------- diff --git a/modules/core/src/main/java/org/apache/ignite/binary/BinaryTypeConfiguration.java b/modules/core/src/main/java/org/apache/ignite/binary/BinaryTypeConfiguration.java index fea0af7..d95e0ae 100644 --- a/modules/core/src/main/java/org/apache/ignite/binary/BinaryTypeConfiguration.java +++ b/modules/core/src/main/java/org/apache/ignite/binary/BinaryTypeConfiguration.java @@ -21,6 +21,7 @@ import org.apache.ignite.configuration.BinaryConfiguration; import org.apache.ignite.configuration.IgniteConfiguration; import org.apache.ignite.internal.util.typedef.internal.A; import org.apache.ignite.internal.util.typedef.internal.S; +import org.jetbrains.annotations.Nullable; /** * Defines configuration properties for a specific binary type. Providing per-type @@ -42,6 +43,9 @@ public class BinaryTypeConfiguration { /** Serializer. */ private BinarySerializer serializer; + /** Identity. */ + private BinaryIdentityResolver identityRslvr; + /** Enum flag. */ private boolean isEnum; @@ -60,10 +64,11 @@ public class BinaryTypeConfiguration { public BinaryTypeConfiguration(BinaryTypeConfiguration other) { A.notNull(other, "other"); - typeName = other.typeName; + identityRslvr = other.identityRslvr; idMapper = other.idMapper; - serializer = other.serializer; isEnum = other.isEnum; + serializer = other.serializer; + typeName = other.typeName; } /** @@ -146,6 +151,24 @@ public class BinaryTypeConfiguration { } /** + * Gets identity resolver. + * + * @return Identity resolver. + */ + @Nullable public BinaryIdentityResolver getIdentityResolver() { + return identityRslvr; + } + + /** + * Sets identity resolver. + * + * @param identityRslvr Identity resolver. + */ + public void setIdentityResolver(@Nullable BinaryIdentityResolver identityRslvr) { + this.identityRslvr = identityRslvr; + } + + /** * Gets whether this is enum type. * * @return {@code True} if enum. http://git-wip-us.apache.org/repos/asf/ignite/blob/86d143bb/modules/core/src/main/java/org/apache/ignite/cache/QueryEntity.java ---------------------------------------------------------------------- diff --git a/modules/core/src/main/java/org/apache/ignite/cache/QueryEntity.java b/modules/core/src/main/java/org/apache/ignite/cache/QueryEntity.java index 9758cfc..fc3b921 100644 --- a/modules/core/src/main/java/org/apache/ignite/cache/QueryEntity.java +++ b/modules/core/src/main/java/org/apache/ignite/cache/QueryEntity.java @@ -20,8 +20,10 @@ package org.apache.ignite.cache; import java.io.Serializable; import java.util.Collection; import java.util.HashMap; +import java.util.HashSet; import java.util.LinkedHashMap; import java.util.Map; +import java.util.Set; import org.apache.ignite.internal.util.typedef.F; import org.apache.ignite.internal.util.typedef.internal.A; @@ -42,6 +44,9 @@ public class QueryEntity implements Serializable { /** Fields available for query. A map from field name to type name. */ private LinkedHashMap fields = new LinkedHashMap<>(); + /** Set of field names that belong to the key. */ + private Set keyFields = new HashSet<>(); + /** Aliases. */ private Map aliases = new HashMap<>(); @@ -123,6 +128,28 @@ public class QueryEntity implements Serializable { } /** + * Gets query fields for this query pair that belongs to the key. We need this for the cases when no key-value classes + * are present on cluster nodes, and we need to build/modify keys and values during SQL DML operations. + * Thus, setting this parameter in XML is not mandatory and should be based on particular use case. + * + * @return Set of names of key fields. + */ + public Set getKeyFields() { + return keyFields; + } + + /** + * Gets query fields for this query pair that belongs to the key. We need this for the cases when no key-value classes + * are present on cluster nodes, and we need to build/modify keys and values during SQL DML operations. + * Thus, setting this parameter in XML is not mandatory and should be based on particular use case. + * + * @param keyFields Set of names of key fields. + */ + public void setKeyFields(Set keyFields) { + this.keyFields = keyFields; + } + + /** * Gets a collection of index entities. * * @return Collection of index entities. http://git-wip-us.apache.org/repos/asf/ignite/blob/86d143bb/modules/core/src/main/java/org/apache/ignite/cache/query/SqlFieldsQuery.java ---------------------------------------------------------------------- diff --git a/modules/core/src/main/java/org/apache/ignite/cache/query/SqlFieldsQuery.java b/modules/core/src/main/java/org/apache/ignite/cache/query/SqlFieldsQuery.java index d3f85af..9b17e78 100644 --- a/modules/core/src/main/java/org/apache/ignite/cache/query/SqlFieldsQuery.java +++ b/modules/core/src/main/java/org/apache/ignite/cache/query/SqlFieldsQuery.java @@ -44,7 +44,7 @@ import org.apache.ignite.internal.util.typedef.internal.S; * * @see IgniteCache#query(Query) */ -public final class SqlFieldsQuery extends Query> { +public class SqlFieldsQuery extends Query> { /** */ private static final long serialVersionUID = 0L; http://git-wip-us.apache.org/repos/asf/ignite/blob/86d143bb/modules/core/src/main/java/org/apache/ignite/configuration/CacheConfiguration.java ---------------------------------------------------------------------- diff --git a/modules/core/src/main/java/org/apache/ignite/configuration/CacheConfiguration.java b/modules/core/src/main/java/org/apache/ignite/configuration/CacheConfiguration.java index f9c114b..56fc5b4 100644 --- a/modules/core/src/main/java/org/apache/ignite/configuration/CacheConfiguration.java +++ b/modules/core/src/main/java/org/apache/ignite/configuration/CacheConfiguration.java @@ -31,6 +31,7 @@ import java.util.HashMap; import java.util.HashSet; import java.util.LinkedHashMap; import java.util.Map; +import java.util.Set; import java.util.TreeSet; import javax.cache.Cache; import javax.cache.CacheException; @@ -2198,7 +2199,7 @@ public class CacheConfiguration extends MutableConfiguration { * @param desc Type descriptor. * @return Type metadata. */ - static QueryEntity convert(TypeDescriptor desc) { + private static QueryEntity convert(TypeDescriptor desc) { QueryEntity entity = new QueryEntity(); // Key and val types. @@ -2208,6 +2209,8 @@ public class CacheConfiguration extends MutableConfiguration { for (ClassProperty prop : desc.props.values()) entity.addQueryField(prop.fullName(), U.box(prop.type()).getName(), prop.alias()); + entity.setKeyFields(desc.keyProperties); + QueryIndex txtIdx = null; Collection idxs = new ArrayList<>(); @@ -2355,7 +2358,7 @@ public class CacheConfiguration extends MutableConfiguration { processAnnotation(key, sqlAnn, txtAnn, field.getType(), prop, type); - type.addProperty(prop, true); + type.addProperty(prop, key, true); } } @@ -2377,7 +2380,7 @@ public class CacheConfiguration extends MutableConfiguration { processAnnotation(key, sqlAnn, txtAnn, mtd.getReturnType(), prop, type); - type.addProperty(prop, true); + type.addProperty(prop, key, true); } } } @@ -2461,6 +2464,10 @@ public class CacheConfiguration extends MutableConfiguration { /** */ @GridToStringInclude + private final Set keyProperties = new HashSet<>(); + + /** */ + @GridToStringInclude private final Map indexes = new HashMap<>(); /** */ @@ -2567,15 +2574,19 @@ public class CacheConfiguration extends MutableConfiguration { * Adds property to the type descriptor. * * @param prop Property. + * @param key Property ownership flag (key or not). * @param failOnDuplicate Fail on duplicate flag. */ - public void addProperty(ClassProperty prop, boolean failOnDuplicate) { + void addProperty(ClassProperty prop, boolean key, boolean failOnDuplicate) { String name = prop.fullName(); if (props.put(name, prop) != null && failOnDuplicate) throw new CacheException("Property with name '" + name + "' already exists."); fields.put(name, prop.type()); + + if (key) + keyProperties.add(name); } /** http://git-wip-us.apache.org/repos/asf/ignite/blob/86d143bb/modules/core/src/main/java/org/apache/ignite/internal/binary/BinaryClassDescriptor.java ---------------------------------------------------------------------- diff --git a/modules/core/src/main/java/org/apache/ignite/internal/binary/BinaryClassDescriptor.java b/modules/core/src/main/java/org/apache/ignite/internal/binary/BinaryClassDescriptor.java index b121337..afe7b37 100644 --- a/modules/core/src/main/java/org/apache/ignite/internal/binary/BinaryClassDescriptor.java +++ b/modules/core/src/main/java/org/apache/ignite/internal/binary/BinaryClassDescriptor.java @@ -751,6 +751,8 @@ public class BinaryClassDescriptor { schemaReg.addSchema(newSchema.schemaId(), newSchema); } } + + postWriteHashCode(writer, obj); } finally { writer.popSchema(); @@ -780,6 +782,7 @@ public class BinaryClassDescriptor { writer.schemaId(stableSchema.schemaId()); postWrite(writer, obj); + postWriteHashCode(writer, obj); } finally { writer.popSchema(); @@ -888,6 +891,18 @@ public class BinaryClassDescriptor { } /** + * Post-write routine for hash code. + * + * @param writer Writer. + * @param obj Object. + */ + private void postWriteHashCode(BinaryWriterExImpl writer, Object obj) { + // No need to call "postWriteHashCode" here because we do not care about hash code. + if (!(obj instanceof CacheObjectImpl)) + writer.postWriteHashCode(registered ? null : cls.getName()); + } + + /** * @return Instance. * @throws BinaryObjectException In case of error. */ http://git-wip-us.apache.org/repos/asf/ignite/blob/86d143bb/modules/core/src/main/java/org/apache/ignite/internal/binary/BinaryContext.java ---------------------------------------------------------------------- diff --git a/modules/core/src/main/java/org/apache/ignite/internal/binary/BinaryContext.java b/modules/core/src/main/java/org/apache/ignite/internal/binary/BinaryContext.java index cc18318..f1f205d 100644 --- a/modules/core/src/main/java/org/apache/ignite/internal/binary/BinaryContext.java +++ b/modules/core/src/main/java/org/apache/ignite/internal/binary/BinaryContext.java @@ -26,6 +26,7 @@ import org.apache.ignite.binary.BinaryIdMapper; import org.apache.ignite.binary.BinaryInvalidTypeException; import org.apache.ignite.binary.BinaryNameMapper; import org.apache.ignite.binary.BinaryObjectException; +import org.apache.ignite.binary.BinaryIdentityResolver; import org.apache.ignite.binary.BinaryReflectiveSerializer; import org.apache.ignite.binary.BinarySerializer; import org.apache.ignite.binary.BinaryType; @@ -135,7 +136,7 @@ public class BinaryContext { /** Set of system classes that should be marshalled with BinaryMarshaller. */ private static final Set BINARYLIZABLE_SYS_CLSS; - /** Binarylizable system classes set initialization. */ + /* Binarylizable system classes set initialization. */ static { Set sysClss = new HashSet<>(); @@ -222,6 +223,9 @@ public class BinaryContext { /** Maps className to mapper */ private final ConcurrentMap cls2Mappers = new ConcurrentHashMap8<>(0); + /** Affinity key field names. */ + private final ConcurrentMap identities = new ConcurrentHashMap8<>(0); + /** */ private BinaryMetadataHandler metaHnd; @@ -433,39 +437,29 @@ public class BinaryContext { throw new BinaryObjectException("Class name is required for binary type configuration."); // Resolve mapper. - BinaryIdMapper idMapper = globalIdMapper; - - if (typeCfg.getIdMapper() != null) - idMapper = typeCfg.getIdMapper(); - - BinaryNameMapper nameMapper = globalNameMapper; - - if (typeCfg.getNameMapper() != null) - nameMapper = typeCfg.getNameMapper(); + BinaryIdMapper idMapper = U.firstNotNull(typeCfg.getIdMapper(), globalIdMapper); + BinaryNameMapper nameMapper = U.firstNotNull(typeCfg.getNameMapper(), globalNameMapper); + BinarySerializer serializer = U.firstNotNull(typeCfg.getSerializer(), globalSerializer); + BinaryIdentityResolver identity = typeCfg.getIdentityResolver(); BinaryInternalMapper mapper = resolveMapper(nameMapper, idMapper); - // Resolve serializer. - BinarySerializer serializer = globalSerializer; - - if (typeCfg.getSerializer() != null) - serializer = typeCfg.getSerializer(); - if (clsName.endsWith(".*")) { String pkgName = clsName.substring(0, clsName.length() - 2); for (String clsName0 : classesInPackage(pkgName)) - descs.add(clsName0, mapper, serializer, affFields.get(clsName0), + descs.add(clsName0, mapper, serializer, identity, affFields.get(clsName0), typeCfg.isEnum(), true); } else - descs.add(clsName, mapper, serializer, affFields.get(clsName), + descs.add(clsName, mapper, serializer, identity, affFields.get(clsName), typeCfg.isEnum(), false); } } for (TypeDescriptor desc : descs.descriptors()) - registerUserType(desc.clsName, desc.mapper, desc.serializer, desc.affKeyFieldName, desc.isEnum); + registerUserType(desc.clsName, desc.mapper, desc.serializer, desc.identity, desc.affKeyFieldName, + desc.isEnum); BinaryInternalMapper globalMapper = resolveMapper(globalNameMapper, globalIdMapper); @@ -1086,6 +1080,7 @@ public class BinaryContext { * @param clsName Class name. * @param mapper ID mapper. * @param serializer Serializer. + * @param identity Type identity. * @param affKeyFieldName Affinity key field name. * @param isEnum If enum. * @throws BinaryObjectException In case of error. @@ -1094,6 +1089,7 @@ public class BinaryContext { public void registerUserType(String clsName, BinaryInternalMapper mapper, @Nullable BinarySerializer serializer, + @Nullable BinaryIdentityResolver identity, @Nullable String affKeyFieldName, boolean isEnum) throws BinaryObjectException { @@ -1114,14 +1110,19 @@ public class BinaryContext { //Workaround for IGNITE-1358 if (predefinedTypes.get(id) != null) - throw new BinaryObjectException("Duplicate type ID [clsName=" + clsName + ", id=" + id + ']'); + throw duplicateTypeIdException(clsName, id); if (typeId2Mapper.put(id, mapper) != null) - throw new BinaryObjectException("Duplicate type ID [clsName=" + clsName + ", id=" + id + ']'); + throw duplicateTypeIdException(clsName, id); + + if (identity != null) { + if (identities.put(id, identity) != null) + throw duplicateTypeIdException(clsName, id); + } if (affKeyFieldName != null) { if (affKeyFieldNames.put(id, affKeyFieldName) != null) - throw new BinaryObjectException("Duplicate type ID [clsName=" + clsName + ", id=" + id + ']'); + throw duplicateTypeIdException(clsName, id); } cls2Mappers.put(clsName, mapper); @@ -1163,6 +1164,16 @@ public class BinaryContext { } /** + * Throw exception on class duplication. + * + * @param clsName Class name. + * @param id Type id. + */ + private static BinaryObjectException duplicateTypeIdException(String clsName, int id) { + return new BinaryObjectException("Duplicate type ID [clsName=" + clsName + ", id=" + id + ']'); + } + + /** * Check whether reflective serializer can be used for class. * * @param cls Class. @@ -1208,6 +1219,14 @@ public class BinaryContext { /** * @param typeId Type ID. + * @return Type identity. + */ + public BinaryIdentityResolver identity(int typeId) { + return identities.get(typeId); + } + + /** + * @param typeId Type ID. * @param meta Meta data. * @throws BinaryObjectException In case of error. */ @@ -1315,6 +1334,7 @@ public class BinaryContext { * @param clsName Class name. * @param mapper Mapper. * @param serializer Serializer. + * @param identity Key hashing mode. * @param affKeyFieldName Affinity key field name. * @param isEnum Enum flag. * @param canOverride Whether this descriptor can be override. @@ -1323,6 +1343,7 @@ public class BinaryContext { private void add(String clsName, BinaryInternalMapper mapper, BinarySerializer serializer, + BinaryIdentityResolver identity, String affKeyFieldName, boolean isEnum, boolean canOverride) @@ -1330,6 +1351,7 @@ public class BinaryContext { TypeDescriptor desc = new TypeDescriptor(clsName, mapper, serializer, + identity, affKeyFieldName, isEnum, canOverride); @@ -1365,6 +1387,9 @@ public class BinaryContext { /** Serializer. */ private BinarySerializer serializer; + /** Type identity. */ + private BinaryIdentityResolver identity; + /** Affinity key field name. */ private String affKeyFieldName; @@ -1376,19 +1401,21 @@ public class BinaryContext { /** * Constructor. - * * @param clsName Class name. * @param mapper ID mapper. * @param serializer Serializer. + * @param identity Key hashing mode. * @param affKeyFieldName Affinity key field name. * @param isEnum Enum type. * @param canOverride Whether this descriptor can be override. */ private TypeDescriptor(String clsName, BinaryInternalMapper mapper, - BinarySerializer serializer, String affKeyFieldName, boolean isEnum, boolean canOverride) { + BinarySerializer serializer, BinaryIdentityResolver identity, String affKeyFieldName, boolean isEnum, + boolean canOverride) { this.clsName = clsName; this.mapper = mapper; this.serializer = serializer; + this.identity = identity; this.affKeyFieldName = affKeyFieldName; this.isEnum = isEnum; this.canOverride = canOverride; @@ -1406,6 +1433,7 @@ public class BinaryContext { if (canOverride) { mapper = other.mapper; serializer = other.serializer; + identity = other.identity; affKeyFieldName = other.affKeyFieldName; isEnum = other.isEnum; canOverride = other.canOverride; http://git-wip-us.apache.org/repos/asf/ignite/blob/86d143bb/modules/core/src/main/java/org/apache/ignite/internal/binary/BinaryFieldImpl.java ---------------------------------------------------------------------- diff --git a/modules/core/src/main/java/org/apache/ignite/internal/binary/BinaryFieldImpl.java b/modules/core/src/main/java/org/apache/ignite/internal/binary/BinaryFieldImpl.java index 78ed17a..59e79fb 100644 --- a/modules/core/src/main/java/org/apache/ignite/internal/binary/BinaryFieldImpl.java +++ b/modules/core/src/main/java/org/apache/ignite/internal/binary/BinaryFieldImpl.java @@ -50,7 +50,6 @@ public class BinaryFieldImpl implements BinaryField { public BinaryFieldImpl(int typeId, BinarySchemaRegistry schemas, String fieldName, int fieldId) { assert typeId != 0; assert schemas != null; - assert fieldName != null; assert fieldId != 0; this.typeId = typeId; @@ -64,6 +63,13 @@ public class BinaryFieldImpl implements BinaryField { return fieldName; } + /** + * @return Field ID. + */ + public int fieldId() { + return fieldId; + } + /** {@inheritDoc} */ @Override public boolean exists(BinaryObject obj) { BinaryObjectExImpl obj0 = (BinaryObjectExImpl)obj; @@ -87,7 +93,7 @@ public class BinaryFieldImpl implements BinaryField { * @param obj Object. * @return Field offset. */ - private int fieldOrder(BinaryObjectExImpl obj) { + public int fieldOrder(BinaryObjectExImpl obj) { if (typeId != obj.typeId()) { throw new BinaryObjectException("Failed to get field because type ID of passed object differs" + " from type ID this " + BinaryField.class.getSimpleName() + " belongs to [expected=" + typeId +