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 4FE18200BD3 for ; Mon, 31 Oct 2016 14:48:05 +0100 (CET) Received: by cust-asf.ponee.io (Postfix) id 4E917160B08; Mon, 31 Oct 2016 13:48:05 +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 4FF20160AF0 for ; Mon, 31 Oct 2016 14:48:03 +0100 (CET) Received: (qmail 83901 invoked by uid 500); 31 Oct 2016 13:48:02 -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 83892 invoked by uid 99); 31 Oct 2016 13:48:02 -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, 31 Oct 2016 13:48:02 +0000 Received: by git1-us-west.apache.org (ASF Mail Server at git1-us-west.apache.org, from userid 33) id 62FAEE0BB1; Mon, 31 Oct 2016 13:48:02 +0000 (UTC) Content-Type: text/plain; charset="us-ascii" MIME-Version: 1.0 Content-Transfer-Encoding: 7bit From: sboikov@apache.org To: commits@ignite.apache.org Date: Mon, 31 Oct 2016 13:48:02 -0000 Message-Id: X-Mailer: ASF-Git Admin Mailer Subject: [01/22] ignite git commit: GG-11360 - Implement SQL queries cancellation (#18) archived-at: Mon, 31 Oct 2016 13:48:05 -0000 Repository: ignite Updated Branches: refs/heads/ignite-4154 [created] 577680984 http://git-wip-us.apache.org/repos/asf/ignite/blob/80abd1b7/modules/indexing/src/test/java/org/apache/ignite/internal/processors/cache/distributed/near/IgniteCacheDistributedQueryStopOnCancelOrTimeoutSelfTest.java ---------------------------------------------------------------------- diff --git a/modules/indexing/src/test/java/org/apache/ignite/internal/processors/cache/distributed/near/IgniteCacheDistributedQueryStopOnCancelOrTimeoutSelfTest.java b/modules/indexing/src/test/java/org/apache/ignite/internal/processors/cache/distributed/near/IgniteCacheDistributedQueryStopOnCancelOrTimeoutSelfTest.java new file mode 100644 index 0000000..f75c907 --- /dev/null +++ b/modules/indexing/src/test/java/org/apache/ignite/internal/processors/cache/distributed/near/IgniteCacheDistributedQueryStopOnCancelOrTimeoutSelfTest.java @@ -0,0 +1,248 @@ +/* + * 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.processors.cache.distributed.near; + +import java.util.Arrays; +import java.util.List; +import java.util.UUID; +import java.util.concurrent.ConcurrentMap; +import java.util.concurrent.TimeUnit; +import javax.cache.CacheException; +import org.apache.ignite.Ignite; +import org.apache.ignite.IgniteCache; +import org.apache.ignite.cache.query.QueryCursor; +import org.apache.ignite.cache.query.SqlFieldsQuery; +import org.apache.ignite.configuration.CacheConfiguration; +import org.apache.ignite.configuration.IgniteConfiguration; +import org.apache.ignite.internal.IgniteEx; +import org.apache.ignite.cache.query.QueryCancelledException; +import org.apache.ignite.internal.processors.query.h2.IgniteH2Indexing; +import org.apache.ignite.internal.util.typedef.G; +import org.apache.ignite.internal.util.typedef.internal.U; +import org.apache.ignite.spi.discovery.tcp.TcpDiscoverySpi; +import org.apache.ignite.spi.discovery.tcp.ipfinder.vm.TcpDiscoveryVmIpFinder; +import org.apache.ignite.testframework.junits.common.GridCommonAbstractTest; + +/** + * Tests distributed SQL queries cancel by user or timeout. + */ +public class IgniteCacheDistributedQueryStopOnCancelOrTimeoutSelfTest extends GridCommonAbstractTest { + /** Grids count. */ + private static final int GRIDS_COUNT = 3; + + /** IP finder. */ + private static final TcpDiscoveryVmIpFinder IP_FINDER = new TcpDiscoveryVmIpFinder(true); + + /** Cache size. */ + public static final int CACHE_SIZE = 10_000; + + /** Value size. */ + public static final int VAL_SIZE = 16; + + /** */ + private static final String QUERY_1 = "select a._val, b._val from String a, String b"; + + /** */ + private static final String QUERY_2 = "select a._key, count(*) from String a group by a._key"; + + /** */ + private static final String QUERY_3 = "select a._val from String a"; + + /** {@inheritDoc} */ + @Override protected void beforeTestsStarted() throws Exception { + super.beforeTestsStarted(); + + startGridsMultiThreaded(GRIDS_COUNT); + } + + /** {@inheritDoc} */ + @Override protected IgniteConfiguration getConfiguration(String gridName) throws Exception { + IgniteConfiguration cfg = super.getConfiguration(gridName); + TcpDiscoverySpi spi = (TcpDiscoverySpi)cfg.getDiscoverySpi(); + spi.setIpFinder(IP_FINDER); + + CacheConfiguration ccfg = new CacheConfiguration<>(); + ccfg.setIndexedTypes(Integer.class, String.class); + + cfg.setCacheConfiguration(ccfg); + + if ("client".equals(gridName)) + cfg.setClientMode(true); + + return cfg; + } + + /** {@inheritDoc} */ + @Override protected void afterTest() throws Exception { + super.afterTest(); + + for (Ignite g : G.allGrids()) + g.cache(null).removeAll(); + } + + /** */ + public void testRemoteQueryExecutionTimeout() throws Exception { + testQuery(CACHE_SIZE, VAL_SIZE, QUERY_1, 500, TimeUnit.MILLISECONDS, true); + } + + /** */ + public void testRemoteQueryWithMergeTableTimeout() throws Exception { + testQuery(CACHE_SIZE, VAL_SIZE, QUERY_2, 500, TimeUnit.MILLISECONDS, true); + } + + /** */ + public void testRemoteQueryExecutionCancel0() throws Exception { + testQuery(CACHE_SIZE, VAL_SIZE, QUERY_1, 1, TimeUnit.MILLISECONDS, false); + } + + /** */ + public void testRemoteQueryExecutionCancel1() throws Exception { + testQuery(CACHE_SIZE, VAL_SIZE, QUERY_1, 500, TimeUnit.MILLISECONDS, false); + } + + /** */ + public void testRemoteQueryExecutionCancel2() throws Exception { + testQuery(CACHE_SIZE, VAL_SIZE, QUERY_1, 1, TimeUnit.SECONDS, false); + } + + /** */ + public void testRemoteQueryExecutionCancel3() throws Exception { + testQuery(CACHE_SIZE, VAL_SIZE, QUERY_1, 3, TimeUnit.SECONDS, false); + } + + /** */ + public void testRemoteQueryWithMergeTableCancel0() throws Exception { + testQuery(CACHE_SIZE, VAL_SIZE, QUERY_2, 1, TimeUnit.MILLISECONDS, false); + } + + /** */ + public void testRemoteQueryWithMergeTableCancel1() throws Exception { + testQuery(CACHE_SIZE, VAL_SIZE, QUERY_2, 500, TimeUnit.MILLISECONDS, false); + } + + /** */ + public void testRemoteQueryWithMergeTableCancel2() throws Exception { + testQuery(CACHE_SIZE, VAL_SIZE, QUERY_2, 1_500, TimeUnit.MILLISECONDS, false); + } + + /** */ + public void testRemoteQueryWithMergeTableCancel3() throws Exception { + testQuery(CACHE_SIZE, VAL_SIZE, QUERY_2, 3, TimeUnit.SECONDS, false); + } + + /** */ + public void testRemoteQueryWithoutMergeTableCancel0() throws Exception { + testQuery(CACHE_SIZE, VAL_SIZE, QUERY_3, 1, TimeUnit.MILLISECONDS, false); + } + + /** */ + public void testRemoteQueryWithoutMergeTableCancel1() throws Exception { + testQuery(CACHE_SIZE, VAL_SIZE, QUERY_3, 500, TimeUnit.MILLISECONDS, false); + } + + /** */ + public void testRemoteQueryWithoutMergeTableCancel2() throws Exception { + testQuery(CACHE_SIZE, VAL_SIZE, QUERY_3, 1_000, TimeUnit.MILLISECONDS, false); + } + + /** */ + public void testRemoteQueryWithoutMergeTableCancel3() throws Exception { + testQuery(CACHE_SIZE, VAL_SIZE, QUERY_3, 3, TimeUnit.SECONDS, false); + } + + /** */ + public void testRemoteQueryAlreadyFinishedStop() throws Exception { + testQuery(100, VAL_SIZE, QUERY_3, 3, TimeUnit.SECONDS, false); + } + + /** */ + private void testQuery(int keyCnt, int valSize, String sql, int timeoutUnits, TimeUnit timeUnit, + boolean timeout) throws Exception { + try (Ignite client = startGrid("client")) { + + IgniteCache cache = client.cache(null); + + assertEquals(0, cache.localSize()); + + int p = 1; + for (int i = 1; i <= keyCnt; i++) { + char[] tmp = new char[valSize]; + Arrays.fill(tmp, ' '); + cache.put(i, new String(tmp)); + + if (i/(float)keyCnt >= p/10f) { + log().info("Loaded " + i + " of " + keyCnt); + + p++; + } + } + + assertEquals(0, cache.localSize()); + + SqlFieldsQuery qry = new SqlFieldsQuery(sql); + + final QueryCursor> cursor; + if (timeout) { + qry.setTimeout(timeoutUnits, timeUnit); + + cursor = cache.query(qry); + } else { + cursor = cache.query(qry); + + client.scheduler().runLocal(new Runnable() { + @Override public void run() { + cursor.close(); + } + }, timeoutUnits, timeUnit); + } + + try(QueryCursor> ignored = cursor) { + cursor.iterator(); + } + catch (CacheException ex) { + log().error("Got expected exception", ex); + + assertTrue("Must throw correct exception", ex.getCause() instanceof QueryCancelledException); + } + + // Give some time to clean up. + Thread.sleep(TimeUnit.MILLISECONDS.convert(timeoutUnits, timeUnit) + 3_000); + + checkCleanState(); + } + } + + /** + * Validates clean state on all participating nodes after query cancellation. + */ + private void checkCleanState() { + for (int i = 0; i < GRIDS_COUNT; i++) { + IgniteEx grid = grid(i); + + // Validate everything was cleaned up. + ConcurrentMap> map = U.field(((IgniteH2Indexing)U.field(U.field( + grid.context(), "qryProc"), "idx")).mapQueryExecutor(), "qryRess"); + + String msg = "Map executor state is not cleared"; + + // TODO FIXME Current implementation leaves map entry for each node that's ever executed a query. + for (ConcurrentMap results : map.values()) + assertEquals(msg, 0, results.size()); + } + } +} \ No newline at end of file http://git-wip-us.apache.org/repos/asf/ignite/blob/80abd1b7/modules/indexing/src/test/java/org/apache/ignite/internal/processors/cache/distributed/near/IgniteCacheQueryNodeRestartSelfTest2.java ---------------------------------------------------------------------- diff --git a/modules/indexing/src/test/java/org/apache/ignite/internal/processors/cache/distributed/near/IgniteCacheQueryNodeRestartSelfTest2.java b/modules/indexing/src/test/java/org/apache/ignite/internal/processors/cache/distributed/near/IgniteCacheQueryNodeRestartSelfTest2.java index 2fa4750..8b33a46 100644 --- a/modules/indexing/src/test/java/org/apache/ignite/internal/processors/cache/distributed/near/IgniteCacheQueryNodeRestartSelfTest2.java +++ b/modules/indexing/src/test/java/org/apache/ignite/internal/processors/cache/distributed/near/IgniteCacheQueryNodeRestartSelfTest2.java @@ -33,7 +33,10 @@ import org.apache.ignite.cache.query.SqlFieldsQuery; import org.apache.ignite.cache.query.annotations.QuerySqlField; import org.apache.ignite.configuration.CacheConfiguration; import org.apache.ignite.configuration.IgniteConfiguration; +import org.apache.ignite.internal.IgniteFutureTimeoutCheckedException; import org.apache.ignite.internal.IgniteInternalFuture; +import org.apache.ignite.internal.IgniteInterruptedCheckedException; +import org.apache.ignite.cache.query.QueryCancelledException; import org.apache.ignite.internal.util.GridRandom; import org.apache.ignite.internal.util.typedef.CAX; import org.apache.ignite.internal.util.typedef.F; @@ -224,58 +227,77 @@ public class IgniteCacheQueryNodeRestartSelfTest2 extends GridCommonAbstractTest } while (!locks.compareAndSet(g, 0, 1)); - if (rnd.nextBoolean()) { // Partitioned query. - IgniteCache cache = grid(g).cache("pu"); + try { + if (rnd.nextBoolean()) { // Partitioned query. + IgniteCache cache = grid(g).cache("pu"); - SqlFieldsQuery qry = new SqlFieldsQuery(PARTITIONED_QRY); + SqlFieldsQuery qry = new SqlFieldsQuery(PARTITIONED_QRY); - boolean smallPageSize = rnd.nextBoolean(); + boolean smallPageSize = rnd.nextBoolean(); - if (smallPageSize) - qry.setPageSize(3); + if (smallPageSize) + qry.setPageSize(3); - try { - assertEquals(pRes, cache.query(qry).getAll()); - } - catch (CacheException e) { - if (!smallPageSize) - e.printStackTrace(); + try { + assertEquals(pRes, cache.query(qry).getAll()); + } catch (CacheException e) { + // Interruptions are expected here. + if (e.getCause() instanceof IgniteInterruptedCheckedException) + continue; - assertTrue("On large page size must retry.", smallPageSize); + if (e.getCause() instanceof QueryCancelledException) + fail("Retry is expected"); - boolean failedOnRemoteFetch = false; + if (!smallPageSize) + e.printStackTrace(); - for (Throwable th = e; th != null; th = th.getCause()) { - if (!(th instanceof CacheException)) - continue; + assertTrue("On large page size must retry.", smallPageSize); + + boolean failedOnRemoteFetch = false; + boolean failedOnInterruption = false; + + for (Throwable th = e; th != null; th = th.getCause()) { + if (th instanceof InterruptedException) { + failedOnInterruption = true; - if (th.getMessage() != null && - th.getMessage().startsWith("Failed to fetch data from node:")) { - failedOnRemoteFetch = true; + break; + } - break; + if (!(th instanceof CacheException)) + continue; + + if (th.getMessage() != null && + th.getMessage().startsWith("Failed to fetch data from node:")) { + failedOnRemoteFetch = true; + + break; + } } - } - if (!failedOnRemoteFetch) { - e.printStackTrace(); + // Interruptions are expected here. + if (failedOnInterruption) + continue; - fail("Must fail inside of GridResultPage.fetchNextPage or subclass."); - } - } - } - else { // Replicated query. - IgniteCache cache = grid(g).cache("co"); + if (!failedOnRemoteFetch) { + e.printStackTrace(); - assertEquals(rRes, cache.query(new SqlFieldsQuery(REPLICATED_QRY)).getAll()); - } + fail("Must fail inside of GridResultPage.fetchNextPage or subclass."); + } + } + } else { // Replicated query. + IgniteCache cache = grid(g).cache("co"); - locks.set(g, 0); + assertEquals(rRes, cache.query(new SqlFieldsQuery(REPLICATED_QRY)).getAll()); + } + } finally { + // Clearing lock in final handler to avoid endless loop if exception is thrown. + locks.set(g, 0); - int c = qryCnt.incrementAndGet(); + int c = qryCnt.incrementAndGet(); - if (c % logFreq == 0) - info("Executed queries: " + c); + if (c % logFreq == 0) + info("Executed queries: " + c); + } } } }, qryThreadNum, "query-thread"); @@ -297,24 +319,26 @@ public class IgniteCacheQueryNodeRestartSelfTest2 extends GridCommonAbstractTest } while (!locks.compareAndSet(g, 0, -1)); - log.info("Stop node: " + g); - - stopGrid(g); + try { + log.info("Stop node: " + g); - Thread.sleep(rnd.nextInt(nodeLifeTime)); + stopGrid(g); - log.info("Start node: " + g); + Thread.sleep(rnd.nextInt(nodeLifeTime)); - startGrid(g); + log.info("Start node: " + g); - Thread.sleep(rnd.nextInt(nodeLifeTime)); + startGrid(g); - locks.set(g, 0); + Thread.sleep(rnd.nextInt(nodeLifeTime)); + } finally { + locks.set(g, 0); - int c = restartCnt.incrementAndGet(); + int c = restartCnt.incrementAndGet(); - if (c % logFreq == 0) - info("Node restarts: " + c); + if (c % logFreq == 0) + info("Node restarts: " + c); + } } return true; @@ -333,7 +357,12 @@ public class IgniteCacheQueryNodeRestartSelfTest2 extends GridCommonAbstractTest qrysDone.set(true); - fut1.get(); + // Query thread can stuck in next page waiting loop because all nodes are left. + try { + fut1.get(5_000); + } catch (IgniteFutureTimeoutCheckedException e) { + fut1.cancel(); + } info("Queries stopped."); } http://git-wip-us.apache.org/repos/asf/ignite/blob/80abd1b7/modules/indexing/src/test/java/org/apache/ignite/internal/processors/cache/local/IgniteCacheLocalQueryCancelOrTimeoutSelfTest.java ---------------------------------------------------------------------- diff --git a/modules/indexing/src/test/java/org/apache/ignite/internal/processors/cache/local/IgniteCacheLocalQueryCancelOrTimeoutSelfTest.java b/modules/indexing/src/test/java/org/apache/ignite/internal/processors/cache/local/IgniteCacheLocalQueryCancelOrTimeoutSelfTest.java new file mode 100644 index 0000000..68ecdff --- /dev/null +++ b/modules/indexing/src/test/java/org/apache/ignite/internal/processors/cache/local/IgniteCacheLocalQueryCancelOrTimeoutSelfTest.java @@ -0,0 +1,158 @@ +/* + * 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.processors.cache.local; + +import java.util.Arrays; +import java.util.List; +import java.util.concurrent.TimeUnit; +import org.apache.ignite.Ignite; +import org.apache.ignite.IgniteCache; +import org.apache.ignite.cache.query.QueryCursor; +import org.apache.ignite.cache.query.SqlFieldsQuery; +import org.apache.ignite.configuration.CacheConfiguration; +import org.apache.ignite.configuration.IgniteConfiguration; +import org.apache.ignite.cache.query.QueryCancelledException; +import org.apache.ignite.internal.util.typedef.G; +import org.apache.ignite.testframework.junits.common.GridCommonAbstractTest; + +import static org.apache.ignite.cache.CacheMode.LOCAL; + +/** + * Tests local query cancellations and timeouts. + */ +public class IgniteCacheLocalQueryCancelOrTimeoutSelfTest extends GridCommonAbstractTest { + /** Cache size. */ + private static final int CACHE_SIZE = 10_000; + + /** */ + private static final String QUERY = "select a._val, b._val from String a, String b"; + + /** {@inheritDoc} */ + @Override protected IgniteConfiguration getConfiguration(String gridName) throws Exception { + IgniteConfiguration cfg = super.getConfiguration(gridName); + + CacheConfiguration ccfg = new CacheConfiguration<>(); + ccfg.setIndexedTypes(Integer.class, String.class); + ccfg.setCacheMode(LOCAL); + + cfg.setCacheConfiguration(ccfg); + + return cfg; + } + + /** {@inheritDoc} */ + @Override protected void beforeTestsStarted() throws Exception { + super.beforeTestsStarted(); + + startGrid(0); + } + + /** {@inheritDoc} */ + @Override protected void afterTest() throws Exception { + super.afterTest(); + + for (Ignite g : G.allGrids()) + g.cache(null).removeAll(); + } + + /** {@inheritDoc} */ + @Override protected void afterTestsStopped() throws Exception { + super.afterTestsStopped(); + + stopAllGrids(); + } + + /** + * @param cache Cache. + */ + private void loadCache(IgniteCache cache) { + int p = 1; + + for (int i = 1; i <= CACHE_SIZE; i++) { + char[] tmp = new char[256]; + Arrays.fill(tmp, ' '); + cache.put(i, new String(tmp)); + + if (i / (float)CACHE_SIZE >= p / 10f) { + log().info("Loaded " + i + " of " + CACHE_SIZE); + + p++; + } + } + } + + /** + * Tests cancellation. + */ + public void testQueryCancel() { + testQuery(false, 1, TimeUnit.SECONDS); + } + + /** + * Tests cancellation with zero timeout. + */ + public void testQueryCancelZeroTimeout() { + testQuery(false, 1, TimeUnit.MILLISECONDS); + } + + /** + * Tests timeout. + */ + public void testQueryTimeout() { + testQuery(true, 1, TimeUnit.SECONDS); + } + + /** + * Tests cancellation. + */ + private void testQuery(boolean timeout, int timeoutUnits, TimeUnit timeUnit) { + Ignite ignite = grid(0); + + IgniteCache cache = ignite.cache(null); + + loadCache(cache); + + SqlFieldsQuery qry = new SqlFieldsQuery(QUERY); + + final QueryCursor> cursor; + if (timeout) { + qry.setTimeout(timeoutUnits, timeUnit); + + cursor = cache.query(qry); + } else { + cursor = cache.query(qry); + + ignite.scheduler().runLocal(new Runnable() { + @Override public void run() { + cursor.close(); + } + }, timeoutUnits, timeUnit); + } + + try(QueryCursor> ignored = cursor) { + cursor.iterator(); + + fail("Expecting timeout"); + } + catch (Exception e) { + assertTrue("Must throw correct exception", e.getCause() instanceof QueryCancelledException); + } + + // Test must exit gracefully. + } +} \ No newline at end of file http://git-wip-us.apache.org/repos/asf/ignite/blob/80abd1b7/modules/indexing/src/test/java/org/apache/ignite/internal/processors/query/h2/GridIndexingSpiAbstractSelfTest.java ---------------------------------------------------------------------- diff --git a/modules/indexing/src/test/java/org/apache/ignite/internal/processors/query/h2/GridIndexingSpiAbstractSelfTest.java b/modules/indexing/src/test/java/org/apache/ignite/internal/processors/query/h2/GridIndexingSpiAbstractSelfTest.java index 0da71c8..289c7d7 100644 --- a/modules/indexing/src/test/java/org/apache/ignite/internal/processors/query/h2/GridIndexingSpiAbstractSelfTest.java +++ b/modules/indexing/src/test/java/org/apache/ignite/internal/processors/query/h2/GridIndexingSpiAbstractSelfTest.java @@ -347,8 +347,8 @@ public abstract class GridIndexingSpiAbstractSelfTest extends GridCommonAbstract // Fields query GridQueryFieldsResult fieldsRes = - spi.queryFields("A", "select a.a.name n1, a.a.age a1, b.a.name n2, " + - "b.a.age a2 from a.a, b.a where a.a.id = b.a.id ", Collections.emptySet(), null); + spi.execute("A", "select a.a.name n1, a.a.age a1, b.a.name n2, " + + "b.a.age a2 from a.a, b.a where a.a.id = b.a.id ", Collections.emptySet(), null, 0, null); String[] aliases = {"N1", "A1", "N2", "A2"}; Object[] vals = { "Valera", 19, "Kolya", 25}; @@ -447,7 +447,7 @@ public abstract class GridIndexingSpiAbstractSelfTest extends GridCommonAbstract time = now; range *= 3; - GridQueryFieldsResult res = spi.queryFields("A", sql, Arrays.asList(1, range), null); + GridQueryFieldsResult res = spi.execute("A", sql, Arrays.asList(1, range), null, 0, null); assert res.iterator().hasNext(); @@ -552,8 +552,7 @@ public abstract class GridIndexingSpiAbstractSelfTest extends GridCommonAbstract return name; } - @Override - public Class type() { + @Override public Class type() { return Object.class; } }; http://git-wip-us.apache.org/repos/asf/ignite/blob/80abd1b7/modules/indexing/src/test/java/org/apache/ignite/testsuites/IgniteCacheQuerySelfTestSuite2.java ---------------------------------------------------------------------- diff --git a/modules/indexing/src/test/java/org/apache/ignite/testsuites/IgniteCacheQuerySelfTestSuite2.java b/modules/indexing/src/test/java/org/apache/ignite/testsuites/IgniteCacheQuerySelfTestSuite2.java index 40fc157..9128f76 100644 --- a/modules/indexing/src/test/java/org/apache/ignite/testsuites/IgniteCacheQuerySelfTestSuite2.java +++ b/modules/indexing/src/test/java/org/apache/ignite/testsuites/IgniteCacheQuerySelfTestSuite2.java @@ -34,11 +34,14 @@ import org.apache.ignite.internal.processors.cache.IgniteCacheP2pUnmarshallingQu import org.apache.ignite.internal.processors.cache.SqlFieldsQuerySelfTest; import org.apache.ignite.internal.processors.cache.distributed.near.IgniteCacheAtomicFieldsQuerySelfTest; import org.apache.ignite.internal.processors.cache.distributed.near.IgniteCacheAtomicNearEnabledFieldsQuerySelfTest; +import org.apache.ignite.internal.processors.cache.distributed.near.IgniteCacheDistributedQueryCancelSelfTest; import org.apache.ignite.internal.processors.cache.distributed.near.IgniteCachePartitionedFieldsQueryP2PEnabledSelfTest; import org.apache.ignite.internal.processors.cache.distributed.near.IgniteCachePartitionedFieldsQuerySelfTest; +import org.apache.ignite.internal.processors.cache.distributed.near.IgniteCacheDistributedQueryStopOnCancelOrTimeoutSelfTest; import org.apache.ignite.internal.processors.cache.distributed.replicated.IgniteCacheReplicatedFieldsQueryP2PEnabledSelfTest; import org.apache.ignite.internal.processors.cache.distributed.replicated.IgniteCacheReplicatedFieldsQuerySelfTest; import org.apache.ignite.internal.processors.cache.local.IgniteCacheLocalFieldsQuerySelfTest; +import org.apache.ignite.internal.processors.cache.local.IgniteCacheLocalQueryCancelOrTimeoutSelfTest; import org.apache.ignite.internal.processors.cache.query.GridCacheSwapScanQuerySelfTest; import org.apache.ignite.internal.processors.cache.reducefields.GridCacheReduceFieldsQueryAtomicSelfTest; import org.apache.ignite.internal.processors.cache.reducefields.GridCacheReduceFieldsQueryLocalSelfTest; @@ -103,6 +106,11 @@ public class IgniteCacheQuerySelfTestSuite2 extends TestSuite { suite.addTestSuite(IgniteCacheP2pUnmarshallingQueryErrorTest.class); suite.addTestSuite(IgniteCacheNoClassQuerySelfTest.class); + // Cancellation. + suite.addTestSuite(IgniteCacheDistributedQueryStopOnCancelOrTimeoutSelfTest.class); + suite.addTestSuite(IgniteCacheDistributedQueryCancelSelfTest.class); + suite.addTestSuite(IgniteCacheLocalQueryCancelOrTimeoutSelfTest.class); + // Other. suite.addTestSuite(CacheQueryNewClientSelfTest.class); suite.addTestSuite(CacheOffheapBatchIndexingSingleTypeTest.class);