You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@ignite.apache.org by sb...@apache.org on 2015/07/30 16:35:33 UTC

[1/8] incubator-ignite git commit: Added test for reproducing problems during simultaneously Ignite instances stopping and cache requests executing

Repository: incubator-ignite
Updated Branches:
  refs/heads/ignite-gg-10295 58f4822b6 -> 7ed4d15f1


Added test for reproducing problems during simultaneously Ignite instances stopping and cache requests executing


Project: http://git-wip-us.apache.org/repos/asf/incubator-ignite/repo
Commit: http://git-wip-us.apache.org/repos/asf/incubator-ignite/commit/0341759c
Tree: http://git-wip-us.apache.org/repos/asf/incubator-ignite/tree/0341759c
Diff: http://git-wip-us.apache.org/repos/asf/incubator-ignite/diff/0341759c

Branch: refs/heads/ignite-gg-10295
Commit: 0341759c2ce8bc342521e7a2a50fe333f06f5a13
Parents: 58f4822
Author: agura <ag...@gridgain.com>
Authored: Fri Jul 24 20:27:03 2015 +0300
Committer: agura <ag...@gridgain.com>
Committed: Fri Jul 24 20:27:03 2015 +0300

----------------------------------------------------------------------
 .../CacheGetFutureHangsSelfTest.java            | 214 +++++++++++++++++++
 1 file changed, 214 insertions(+)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/0341759c/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/CacheGetFutureHangsSelfTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/CacheGetFutureHangsSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/CacheGetFutureHangsSelfTest.java
new file mode 100644
index 0000000..e2acb2e
--- /dev/null
+++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/CacheGetFutureHangsSelfTest.java
@@ -0,0 +1,214 @@
+/*
+ * 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;
+
+import org.apache.ignite.*;
+import org.apache.ignite.cache.*;
+import org.apache.ignite.configuration.*;
+import org.apache.ignite.internal.*;
+import org.apache.ignite.internal.util.typedef.internal.*;
+import org.apache.ignite.marshaller.optimized.*;
+import org.apache.ignite.testframework.junits.common.*;
+
+import java.util.*;
+import java.util.concurrent.*;
+import java.util.concurrent.atomic.*;
+
+import static org.apache.ignite.cache.CacheWriteSynchronizationMode.*;
+
+/**
+ * Test for reproducing problems during simultaneously Ignite instances stopping and cache requests executing.
+ */
+public class CacheGetFutureHangsSelfTest extends GridCommonAbstractTest {
+    /** Grid count. */
+    private static final int GRID_CNT = 8;
+
+    /** Grids. */
+    private static Ignite[] grids;
+
+    /** Ids. */
+    private static String[] ids;
+
+    /** Flags. */
+    private static AtomicBoolean[] flags;
+
+    /** Futs. */
+    private static Collection<IgniteInternalFuture> futs;
+
+    /** Alive grids. */
+    private static Set<Integer> aliveGrids;
+
+    /** {@inheritDoc} */
+    @Override protected IgniteConfiguration getConfiguration(String gridName) throws Exception {
+        IgniteConfiguration cfg = super.getConfiguration(gridName);
+
+        OptimizedMarshaller marsh = new OptimizedMarshaller();
+        marsh.setRequireSerializable(false);
+
+        cfg.setMarshaller(marsh);
+
+        CacheConfiguration ccfg = defaultCacheConfiguration();
+        ccfg.setCacheMode(CacheMode.PARTITIONED);
+        ccfg.setBackups(1);
+        ccfg.setAtomicityMode(CacheAtomicityMode.ATOMIC);
+        ccfg.setWriteSynchronizationMode(PRIMARY_SYNC);
+        ccfg.setNearConfiguration(null);
+
+        cfg.setCacheConfiguration(ccfg);
+
+        return cfg;
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testFailover() throws Exception {
+        int cnt = 10;
+
+        for (int i = 0; i < cnt; i++) {
+            try {
+                U.debug("*** Iteration " + (i + 1) + '/' + cnt);
+
+                init();
+
+                doTestFailover();
+            }
+            finally {
+                stopAllGrids();
+            }
+        }
+    }
+
+    /**
+     * Initializes test.
+     */
+    private void init() {
+        grids = new Ignite[GRID_CNT + 1];
+
+        ids = new String[GRID_CNT + 1];
+
+        aliveGrids = new HashSet<>();
+
+        flags = new AtomicBoolean[GRID_CNT + 1];
+
+        futs = new ArrayList<>();
+    }
+
+    /**
+     * Executes one test iteration.
+     */
+    private void doTestFailover() throws Exception {
+        try {
+            for (int i = 0; i < GRID_CNT + 1; i++) {
+                final IgniteEx grid = startGrid(i);
+
+                grids[i] = grid;
+
+                ids[i] = grid.localNode().id().toString();
+
+                aliveGrids.add(i);
+
+                flags[i] = new AtomicBoolean();
+            }
+
+            for (int i = 0; i < GRID_CNT + 1; i++) {
+                final int gridIdx = i;
+
+                futs.add(multithreadedAsync(new Runnable() {
+                    @Override public void run() {
+                        IgniteCache cache = grids[gridIdx].cache(null);
+
+                        while (!flags[gridIdx].get()) {
+                            int idx = ThreadLocalRandom.current().nextInt(GRID_CNT + 1);
+
+                            String id = ids[idx];
+
+                            if (id != null /*&& grids[gridIdx] != null*/) {
+                                //U.debug("!!! Grid containsKey start " + gridIdx);
+                                cache.containsKey(id);
+                                //U.debug("!!! Grid containsKey finished " + gridIdx);
+                            }
+
+                            try {
+                                Thread.sleep(ThreadLocalRandom.current().nextLong(50));
+                            }
+                            catch (InterruptedException e) {
+                                Thread.currentThread().interrupt();
+                            }
+                        }
+                    }
+                }, 1, "containsKey-thread-" + i));
+
+                futs.add(multithreadedAsync(new Runnable() {
+                    @Override public void run() {
+                        IgniteCache cache = grids[gridIdx].cache(null);
+
+                        while (!flags[gridIdx].get()) {
+                            int idx = ThreadLocalRandom.current().nextInt(GRID_CNT + 1);
+
+                            String id = ids[idx];
+
+                            if (id != null /*&& grids[gridIdx] != null*/) {
+                                //U.debug("!!! Grid put start " + gridIdx);
+                                cache.put(id, UUID.randomUUID());
+                                //U.debug("!!! Grid put finished " + gridIdx);
+                            }
+
+                            try {
+                                Thread.sleep(ThreadLocalRandom.current().nextLong(50));
+                            }
+                            catch (InterruptedException e) {
+                                Thread.currentThread().interrupt();
+                            }
+                        }
+                    }
+                }, 1, "put-thread-" + i));
+            }
+
+            while (aliveGrids.size() > 1) {
+                final int gridToKill = ThreadLocalRandom.current().nextInt(GRID_CNT) + 1;
+
+                if (gridToKill > 0 && grids[gridToKill] != null) {
+                    U.debug("!!! Trying to kill grid " + gridToKill);
+
+                    //synchronized (mons[gridToKill]) {
+                        U.debug("!!! Grid stop start " + gridToKill);
+
+                        grids[gridToKill].close();
+
+                        aliveGrids.remove(gridToKill);
+
+                        grids[gridToKill] = null;
+
+                        flags[gridToKill].set(true);
+
+                        U.debug("!!! Grid stop finished " + gridToKill);
+                    //}
+                }
+            }
+
+            Thread.sleep(ThreadLocalRandom.current().nextLong(100));
+        }
+        finally {
+            flags[0].set(true);
+
+            for (IgniteInternalFuture fut : futs)
+                fut.get();
+        }
+    }
+}


[8/8] incubator-ignite git commit: #ignite-1175: Add test for dht local partition map.

Posted by sb...@apache.org.
#ignite-1175: Add test for dht local partition map.


Project: http://git-wip-us.apache.org/repos/asf/incubator-ignite/repo
Commit: http://git-wip-us.apache.org/repos/asf/incubator-ignite/commit/7ed4d15f
Tree: http://git-wip-us.apache.org/repos/asf/incubator-ignite/tree/7ed4d15f
Diff: http://git-wip-us.apache.org/repos/asf/incubator-ignite/diff/7ed4d15f

Branch: refs/heads/ignite-gg-10295
Commit: 7ed4d15f16c71e1683fd659865653a383d99259e
Parents: 5288b2d
Author: ivasilinets <iv...@gridgain.com>
Authored: Thu Jul 30 14:12:27 2015 +0300
Committer: ivasilinets <iv...@gridgain.com>
Committed: Thu Jul 30 14:12:27 2015 +0300

----------------------------------------------------------------------
 ...cheDhtLocalPartitionAfterRemoveSelfTest.java | 107 +++++++++++++++++++
 1 file changed, 107 insertions(+)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/7ed4d15f/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheDhtLocalPartitionAfterRemoveSelfTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheDhtLocalPartitionAfterRemoveSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheDhtLocalPartitionAfterRemoveSelfTest.java
new file mode 100644
index 0000000..b04e41a
--- /dev/null
+++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheDhtLocalPartitionAfterRemoveSelfTest.java
@@ -0,0 +1,107 @@
+/*
+ * 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;
+
+import org.apache.ignite.*;
+import org.apache.ignite.cache.*;
+import org.apache.ignite.configuration.*;
+import org.apache.ignite.internal.processors.cache.distributed.dht.*;
+import org.apache.ignite.testframework.junits.common.*;
+
+/**
+ * Test for remove operation.
+ */
+public class CacheDhtLocalPartitionAfterRemoveSelfTest extends GridCommonAbstractTest {
+    /** {@inheritDoc} */
+    @Override protected IgniteConfiguration getConfiguration(String gridName) throws Exception {
+        IgniteConfiguration cfg = super.getConfiguration(gridName);
+
+        CacheConfiguration ccfg = new CacheConfiguration();
+
+        ccfg.setAtomicityMode(CacheAtomicityMode.TRANSACTIONAL);
+        ccfg.setEvictSynchronized(false);
+        ccfg.setNearConfiguration(null);
+
+        cfg.setCacheConfiguration(ccfg);
+
+        return cfg;
+    }
+
+    /** {@inheritDoc} */
+    @Override protected void beforeTestsStarted() throws Exception {
+        startGrids(1);
+    }
+
+    /** {@inheritDoc} */
+    @Override protected void afterTestsStopped() throws Exception {
+        stopAllGrids();
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testMemoryUsage() throws Exception {
+        IgniteCache<TestKey, Integer> cache = grid(0).cache(null);
+
+        for (int i = 0; i < 1000; ++i)
+            cache.put(new TestKey("" + i), i);
+
+        for (int i = 0; i < 1000; ++i)
+            assert cache.getAndRemove(new TestKey("" + i)).equals(i);
+
+        assertEquals(0, cache.size());
+
+        int size = 0;
+
+        for (GridDhtLocalPartition p : dht(cache).topology().localPartitions()) {
+            int pSize = p.size();
+
+            size += pSize;
+        }
+
+        System.out.println("All size: " + size);
+    }
+
+    /**
+     * Test key.
+     */
+    private static class TestKey {
+        /** Key. */
+        private String key;
+
+        /**
+         * @param key Key.
+         */
+        public TestKey(String key) {
+            this.key = key;
+        }
+
+        /** {@inheritDoc} */
+        @Override public int hashCode() {
+            return key.hashCode();
+        }
+
+        /** {@inheritDoc} */
+        @Override public boolean equals(Object obj) {
+            if (obj == null || !(obj instanceof TestKey))
+                return false;
+
+            return key.equals(((TestKey)obj).key);
+        }
+    }
+}


[2/8] incubator-ignite git commit: #ignite-961: add new rest commands.

Posted by sb...@apache.org.
#ignite-961:  add new rest commands.


Project: http://git-wip-us.apache.org/repos/asf/incubator-ignite/repo
Commit: http://git-wip-us.apache.org/repos/asf/incubator-ignite/commit/9bfc1842
Tree: http://git-wip-us.apache.org/repos/asf/incubator-ignite/tree/9bfc1842
Diff: http://git-wip-us.apache.org/repos/asf/incubator-ignite/diff/9bfc1842

Branch: refs/heads/ignite-gg-10295
Commit: 9bfc1842a20c6dc2f55f133a997608dd1b5f803a
Parents: 0341759
Author: ivasilinets <iv...@gridgain.com>
Authored: Mon Jul 27 10:36:46 2015 +0300
Committer: ivasilinets <iv...@gridgain.com>
Committed: Mon Jul 27 10:36:46 2015 +0300

----------------------------------------------------------------------
 .../rest/AbstractRestProcessorSelfTest.java     |   1 +
 .../JettyRestProcessorAbstractSelfTest.java     | 616 ++++++++++++++++++-
 .../apache/ignite/internal/IgniteKernal.java    |  42 +-
 .../processors/cache/GridCacheAdapter.java      |   2 +-
 .../processors/rest/GridRestCommand.java        |  53 +-
 .../processors/rest/GridRestProcessor.java      |  32 +
 .../handlers/cache/GridCacheCommandHandler.java | 373 ++++++++++-
 .../rest/handlers/query/CacheQueryResult.java   | 101 +++
 .../handlers/query/QueryCommandHandler.java     | 271 ++++++++
 .../version/GridVersionCommandHandler.java      |  14 +-
 .../rest/request/RestSqlQueryRequest.java       | 125 ++++
 .../http/jetty/GridJettyRestHandler.java        |  85 ++-
 12 files changed, 1639 insertions(+), 76 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/9bfc1842/modules/clients/src/test/java/org/apache/ignite/internal/processors/rest/AbstractRestProcessorSelfTest.java
----------------------------------------------------------------------
diff --git a/modules/clients/src/test/java/org/apache/ignite/internal/processors/rest/AbstractRestProcessorSelfTest.java b/modules/clients/src/test/java/org/apache/ignite/internal/processors/rest/AbstractRestProcessorSelfTest.java
index 4f1969f..8310b0f 100644
--- a/modules/clients/src/test/java/org/apache/ignite/internal/processors/rest/AbstractRestProcessorSelfTest.java
+++ b/modules/clients/src/test/java/org/apache/ignite/internal/processors/rest/AbstractRestProcessorSelfTest.java
@@ -84,6 +84,7 @@ abstract class AbstractRestProcessorSelfTest extends GridCommonAbstractTest {
         CacheConfiguration ccfg = defaultCacheConfiguration();
 
         ccfg.setStatisticsEnabled(true);
+        ccfg.setIndexedTypes(String.class, String.class);
 
         cfg.setCacheConfiguration(ccfg);
 

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/9bfc1842/modules/clients/src/test/java/org/apache/ignite/internal/processors/rest/JettyRestProcessorAbstractSelfTest.java
----------------------------------------------------------------------
diff --git a/modules/clients/src/test/java/org/apache/ignite/internal/processors/rest/JettyRestProcessorAbstractSelfTest.java b/modules/clients/src/test/java/org/apache/ignite/internal/processors/rest/JettyRestProcessorAbstractSelfTest.java
index d5a3cc1..160046c 100644
--- a/modules/clients/src/test/java/org/apache/ignite/internal/processors/rest/JettyRestProcessorAbstractSelfTest.java
+++ b/modules/clients/src/test/java/org/apache/ignite/internal/processors/rest/JettyRestProcessorAbstractSelfTest.java
@@ -17,12 +17,20 @@
 
 package org.apache.ignite.internal.processors.rest;
 
+import net.sf.json.*;
+import org.apache.ignite.*;
 import org.apache.ignite.cache.*;
+import org.apache.ignite.cache.query.*;
+import org.apache.ignite.cache.query.annotations.*;
+import org.apache.ignite.configuration.*;
+import org.apache.ignite.internal.processors.rest.handlers.*;
 import org.apache.ignite.internal.util.typedef.*;
+import org.apache.ignite.testframework.*;
 
 import java.io.*;
 import java.net.*;
 import java.util.*;
+import java.util.concurrent.*;
 import java.util.regex.*;
 
 import static org.apache.ignite.IgniteSystemProperties.*;
@@ -31,7 +39,7 @@ import static org.apache.ignite.IgniteSystemProperties.*;
  * Tests for Jetty REST protocol.
  */
 @SuppressWarnings("unchecked")
-abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorSelfTest {
+public abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorSelfTest {
     /** Grid count. */
     private static final int GRID_CNT = 3;
 
@@ -40,6 +48,8 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
         System.setProperty(IGNITE_JETTY_PORT, Integer.toString(restPort()));
 
         super.beforeTestsStarted();
+
+        initCache();
     }
 
     /** {@inheritDoc} */
@@ -50,6 +60,11 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
     }
 
     /** {@inheritDoc} */
+    @Override protected void beforeTest() throws Exception {
+        grid(0).cache(null).removeAll();
+    }
+
+    /** {@inheritDoc} */
     @Override protected int gridCount() {
         return GRID_CNT;
     }
@@ -65,7 +80,7 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
      * @return Returned content.
      * @throws Exception If failed.
      */
-    private String content(Map<String, String> params) throws Exception {
+    protected String content(Map<String, String> params) throws Exception {
         String addr = "http://" + LOC_HOST + ":" + restPort() + "/ignite?";
 
         for (Map.Entry<String, String> e : params.entrySet())
@@ -115,6 +130,18 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
     }
 
     /**
+     * @param err Error.
+     * @return Regex pattern for JSON.
+     */
+    private String errorPattern(String err) {
+        return "\\{" +
+            "\\\"error\\\":\\\"" + err + "\\\"\\," +
+            "\\\"response\\\":null\\," +
+            "\\\"sessionToken\\\":\\\"\\\"," +
+            "\\\"successStatus\\\":" + 1 + "\\}";
+    }
+
+    /**
      * @param res Response.
      * @param success Success flag.
      * @return Regex pattern for JSON.
@@ -144,6 +171,19 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
      * @param success Success flag.
      * @return Regex pattern for JSON.
      */
+    private String cacheBulkPattern(int res, boolean success) {
+        return "\\{\\\"affinityNodeId\\\":\\\"\\\"\\," +
+            "\\\"error\\\":\\\"\\\"\\," +
+            "\\\"response\\\":" + res + "\\," +
+            "\\\"sessionToken\\\":\\\"\\\"," +
+            "\\\"successStatus\\\":" + (success ? 0 : 1) + "\\}";
+    }
+
+    /**
+     * @param res Response.
+     * @param success Success flag.
+     * @return Regex pattern for JSON.
+     */
     private String cachePattern(boolean res, boolean success) {
         return "\\{\\\"affinityNodeId\\\":\\\"\\w{8}-\\w{4}-\\w{4}-\\w{4}-\\w{12}\\\"\\," +
             "\\\"error\\\":\\\"\\\"\\," +
@@ -208,7 +248,7 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
     public void testGet() throws Exception {
         jcache().put("getKey", "getVal");
 
-        String ret = content(F.asMap("cmd", "get", "key", "getKey"));
+        String ret = content(F.asMap("cmd", GridRestCommand.CACHE_GET.key(), "key", "getKey"));
 
         assertNotNull(ret);
         assertTrue(!ret.isEmpty());
@@ -221,11 +261,64 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
     /**
      * @throws Exception If failed.
      */
+    public void testCacheSize() throws Exception {
+        jcache().removeAll();
+
+        jcache().put("getKey", "getVal");
+
+        String ret = content(F.asMap("cmd", GridRestCommand.CACHE_SIZE.key()));
+
+        assertNotNull(ret);
+        assertTrue(!ret.isEmpty());
+
+        info("Size command result: " + ret);
+
+        jsonEquals(ret, cacheBulkPattern(1, true));
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testIgniteName() throws Exception {
+        String ret = content(F.asMap("cmd", GridRestCommand.NAME.key()));
+
+        assertNotNull(ret);
+        assertTrue(!ret.isEmpty());
+
+        info("Name command result: " + ret);
+
+        jsonEquals(ret, stringPattern(getTestGridName(0), true));
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testGetOrCreateCache() throws Exception {
+        String ret = content(F.asMap("cmd", GridRestCommand.GET_OR_CREATE_CACHE.key(), "cacheName", "testCache"));
+
+        assertNotNull(ret);
+        assertTrue(!ret.isEmpty());
+
+        info("Name command result: " + ret);
+
+        grid(0).cache("testCache").put("1", "1");
+
+        ret = content(F.asMap("cmd", GridRestCommand.DESTROY_CACHE.key(), "cacheName", "testCache"));
+
+        assertNotNull(ret);
+        assertTrue(!ret.isEmpty());
+
+        assertNull(grid(0).cache("testCache"));
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
     public void testGetAll() throws Exception {
         jcache().put("getKey1", "getVal1");
         jcache().put("getKey2", "getVal2");
 
-        String ret = content(F.asMap("cmd", "getall", "k1", "getKey1", "k2", "getKey2"));
+        String ret = content(F.asMap("cmd", GridRestCommand.CACHE_GET_ALL.key(), "k1", "getKey1", "k2", "getKey2"));
 
         assertNotNull(ret);
         assertTrue(!ret.isEmpty());
@@ -235,14 +328,192 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
         jsonEquals(ret,
             // getKey[12] is used since the order is not determined.
             cacheBulkPattern("\\{\\\"getKey[12]\\\":\\\"getVal[12]\\\"\\,\\\"getKey[12]\\\":\\\"getVal[12]\\\"\\}",
-            true));
+                true));
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testIncorrectPut() throws Exception {
+        String ret = content(F.asMap("cmd", GridRestCommand.CACHE_PUT.key(), "key", "key0"));
+
+        assertNotNull(ret);
+        assertTrue(!ret.isEmpty());
+        jsonEquals(ret, errorPattern("Failed to find mandatory parameter in request: val"));
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testContainsKey() throws Exception {
+        grid(0).cache(null).put("key0", "val0");
+
+        String ret = content(F.asMap("cmd", GridRestCommand.CACHE_CONTAINS_KEY.key(), "key", "key0"));
+
+        assertNotNull(ret);
+        assertTrue(!ret.isEmpty());
+
+        jsonEquals(ret, cachePattern(true, true));
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testContainesKeys() throws Exception {
+        grid(0).cache(null).put("key0", "val0");
+        grid(0).cache(null).put("key1", "val1");
+
+        String ret = content(F.asMap("cmd", GridRestCommand.CACHE_CONTAINS_KEYS.key(),
+            "k1", "key0", "k2", "key1"));
+
+        assertNotNull(ret);
+        assertTrue(!ret.isEmpty());
+
+        jsonEquals(ret, cacheBulkPattern(true, true));
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testGetAndPut() throws Exception {
+        grid(0).cache(null).put("key0", "val0");
+
+        String ret = content(F.asMap("cmd", GridRestCommand.CACHE_GET_AND_PUT.key(), "key", "key0", "val", "val1"));
+
+        assertNotNull(ret);
+        assertTrue(!ret.isEmpty());
+
+        jsonEquals(ret, cachePattern("val0", true));
+
+        assertEquals("val1", grid(0).cache(null).get("key0"));
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testGetAndPutIfAbsent() throws Exception {
+        grid(0).cache(null).put("key0", "val0");
+
+        String ret = content(F.asMap("cmd", GridRestCommand.CACHE_GET_AND_PUT_IF_ABSENT.key(),
+            "key", "key0", "val", "val1"));
+
+        assertNotNull(ret);
+        assertTrue(!ret.isEmpty());
+
+        jsonEquals(ret, cachePattern("val0", true));
+
+        assertEquals("val0", grid(0).cache(null).get("key0"));
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testPutIfAbsent2() throws Exception {
+        String ret = content(F.asMap("cmd", GridRestCommand.CACHE_PUT_IF_ABSENT.key(),
+            "key", "key0", "val", "val1"));
+
+        assertNotNull(ret);
+        assertTrue(!ret.isEmpty());
+
+        jsonEquals(ret, cachePattern(true, true));
+
+        assertEquals("val1", grid(0).cache(null).get("key0"));
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testRemoveValue() throws Exception {
+        grid(0).cache(null).put("key0", "val0");
+
+        String ret = content(F.asMap("cmd", GridRestCommand.CACHE_REMOVE_VALUE.key(),
+            "key", "key0", "val", "val1"));
+
+        assertNotNull(ret);
+        assertTrue(!ret.isEmpty());
+
+        jsonEquals(ret, cachePattern(false, true));
+
+        assertEquals("val0", grid(0).cache(null).get("key0"));
+
+        ret = content(F.asMap("cmd", GridRestCommand.CACHE_REMOVE_VALUE.key(),
+            "key", "key0", "val", "val0"));
+
+        assertNotNull(ret);
+        assertTrue(!ret.isEmpty());
+
+        jsonEquals(ret, cachePattern(true, true));
+
+        assertNull(grid(0).cache(null).get("key0"));
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testGetAndRemove() throws Exception {
+        grid(0).cache(null).put("key0", "val0");
+
+        String ret = content(F.asMap("cmd", GridRestCommand.CACHE_GET_AND_REMOVE.key(),
+            "key", "key0"));
+
+        assertNotNull(ret);
+        assertTrue(!ret.isEmpty());
+
+        jsonEquals(ret, cachePattern("val0", true));
+
+        assertNull(grid(0).cache(null).get("key0"));
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testReplaceValue() throws Exception {
+        grid(0).cache(null).put("key0", "val0");
+
+        String ret = content(F.asMap("cmd", GridRestCommand.CACHE_REPLACE_VALUE.key(),
+            "key", "key0", "val", "val1", "val2", "val2"));
+
+        assertNotNull(ret);
+        assertTrue(!ret.isEmpty());
+
+        jsonEquals(ret, cachePattern(false, true));
+
+        assertEquals("val0", grid(0).cache(null).get("key0"));
+
+        ret = content(F.asMap("cmd", GridRestCommand.CACHE_REPLACE_VALUE.key(),
+            "key", "key0", "val", "val1", "val2", "val0"));
+
+        assertNotNull(ret);
+        assertTrue(!ret.isEmpty());
+
+        jsonEquals(ret, cachePattern(true, true));
+
+        assertEquals("val1", grid(0).cache(null).get("key0"));
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testGetAndReplace() throws Exception {
+        grid(0).cache(null).put("key0", "val0");
+
+        String ret = content(F.asMap("cmd", GridRestCommand.CACHE_GET_AND_REPLACE.key(),
+            "key", "key0", "val", "val1"));
+
+        assertNotNull(ret);
+        assertTrue(!ret.isEmpty());
+
+        jsonEquals(ret, cachePattern("val0", true));
+
+        assertEquals("val1", grid(0).cache(null).get("key0"));
     }
 
     /**
      * @throws Exception If failed.
      */
     public void testPut() throws Exception {
-        String ret = content(F.asMap("cmd", "put", "key", "putKey", "val", "putVal"));
+        String ret = content(F.asMap("cmd", GridRestCommand.CACHE_PUT.key(),
+            "key", "putKey", "val", "putVal"));
 
         assertNotNull(ret);
         assertTrue(!ret.isEmpty());
@@ -258,7 +529,8 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
      * @throws Exception If failed.
      */
     public void testPutWithExpiration() throws Exception {
-        String ret = content(F.asMap("cmd", "put", "key", "putKey", "val", "putVal", "exp", "2000"));
+        String ret = content(F.asMap("cmd", GridRestCommand.CACHE_PUT.key(),
+            "key", "putKey", "val", "putVal", "exp", "2000"));
 
         assertNotNull(ret);
         assertTrue(!ret.isEmpty());
@@ -278,7 +550,8 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
     public void testAdd() throws Exception {
         jcache().put("addKey1", "addVal1");
 
-        String ret = content(F.asMap("cmd", "add", "key", "addKey2", "val", "addVal2"));
+        String ret = content(F.asMap("cmd", GridRestCommand.CACHE_ADD.key(),
+            "key", "addKey2", "val", "addVal2"));
 
         assertNotNull(ret);
         assertTrue(!ret.isEmpty());
@@ -293,7 +566,8 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
      * @throws Exception If failed.
      */
     public void testAddWithExpiration() throws Exception {
-        String ret = content(F.asMap("cmd", "add", "key", "addKey", "val", "addVal", "exp", "2000"));
+        String ret = content(F.asMap("cmd", GridRestCommand.CACHE_ADD.key(),
+            "key", "addKey", "val", "addVal", "exp", "2000"));
 
         assertNotNull(ret);
         assertTrue(!ret.isEmpty());
@@ -311,7 +585,8 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
      * @throws Exception If failed.
      */
     public void testPutAll() throws Exception {
-        String ret = content(F.asMap("cmd", "putall", "k1", "putKey1", "k2", "putKey2",
+        String ret = content(F.asMap("cmd", GridRestCommand.CACHE_PUT_ALL.key(),
+            "k1", "putKey1", "k2", "putKey2",
             "v1", "putVal1", "v2", "putVal2"));
 
         assertNotNull(ret);
@@ -333,7 +608,8 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
 
         assertEquals("rmvVal", jcache().localPeek("rmvKey", CachePeekMode.ONHEAP));
 
-        String ret = content(F.asMap("cmd", "rmv", "key", "rmvKey"));
+        String ret = content(F.asMap("cmd", GridRestCommand.CACHE_REMOVE.key(),
+            "key", "rmvKey"));
 
         assertNotNull(ret);
         assertTrue(!ret.isEmpty());
@@ -359,7 +635,8 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
         assertEquals("rmvVal3", jcache().localPeek("rmvKey3", CachePeekMode.ONHEAP));
         assertEquals("rmvVal4", jcache().localPeek("rmvKey4", CachePeekMode.ONHEAP));
 
-        String ret = content(F.asMap("cmd", "rmvall", "k1", "rmvKey1", "k2", "rmvKey2"));
+        String ret = content(F.asMap("cmd", GridRestCommand.CACHE_REMOVE_ALL.key(),
+            "k1", "rmvKey1", "k2", "rmvKey2"));
 
         assertNotNull(ret);
         assertTrue(!ret.isEmpty());
@@ -397,7 +674,8 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
 
         assertEquals("casOldVal", jcache().localPeek("casKey", CachePeekMode.ONHEAP));
 
-        String ret = content(F.asMap("cmd", "cas", "key", "casKey", "val2", "casOldVal", "val1", "casNewVal"));
+        String ret = content(F.asMap("cmd", GridRestCommand.CACHE_CAS.key(),
+            "key", "casKey", "val2", "casOldVal", "val1", "casNewVal"));
 
         assertNotNull(ret);
         assertTrue(!ret.isEmpty());
@@ -419,7 +697,8 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
 
         assertEquals("repOldVal", jcache().localPeek("repKey", CachePeekMode.ONHEAP));
 
-        String ret = content(F.asMap("cmd", "rep", "key", "repKey", "val", "repVal"));
+        String ret = content(F.asMap("cmd", GridRestCommand.CACHE_REPLACE.key(),
+            "key", "repKey", "val", "repVal"));
 
         assertNotNull(ret);
         assertTrue(!ret.isEmpty());
@@ -439,7 +718,8 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
 
         assertEquals("replaceVal", jcache().get("replaceKey"));
 
-        String ret = content(F.asMap("cmd", "rep", "key", "replaceKey", "val", "replaceValNew", "exp", "2000"));
+        String ret = content(F.asMap("cmd", GridRestCommand.CACHE_REPLACE.key(),
+            "key", "replaceKey", "val", "replaceValNew", "exp", "2000"));
 
         assertNotNull(ret);
         assertTrue(!ret.isEmpty());
@@ -460,7 +740,8 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
     public void testAppend() throws Exception {
         jcache().put("appendKey", "appendVal");
 
-        String ret = content(F.asMap("cmd", "append", "key", "appendKey", "val", "_suffix"));
+        String ret = content(F.asMap("cmd", GridRestCommand.CACHE_APPEND.key(),
+            "key", "appendKey", "val", "_suffix"));
 
         assertNotNull(ret);
         assertTrue(!ret.isEmpty());
@@ -476,7 +757,8 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
     public void testPrepend() throws Exception {
         jcache().put("prependKey", "prependVal");
 
-        String ret = content(F.asMap("cmd", "prepend", "key", "prependKey", "val", "prefix_"));
+        String ret = content(F.asMap("cmd", GridRestCommand.CACHE_PREPEND.key(),
+            "key", "prependKey", "val", "prefix_"));
 
         assertNotNull(ret);
         assertTrue(!ret.isEmpty());
@@ -490,7 +772,8 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
      * @throws Exception If failed.
      */
     public void testIncrement() throws Exception {
-        String ret = content(F.asMap("cmd", "incr", "key", "incrKey", "init", "2", "delta", "3"));
+        String ret = content(F.asMap("cmd", GridRestCommand.ATOMIC_INCREMENT.key(),
+            "key", "incrKey", "init", "2", "delta", "3"));
 
         assertNotNull(ret);
         assertTrue(!ret.isEmpty());
@@ -499,7 +782,7 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
 
         assertEquals(5, grid(0).atomicLong("incrKey", 0, true).get());
 
-        ret = content(F.asMap("cmd", "incr", "key", "incrKey", "delta", "10"));
+        ret = content(F.asMap("cmd", GridRestCommand.ATOMIC_INCREMENT.key(), "key", "incrKey", "delta", "10"));
 
         assertNotNull(ret);
         assertTrue(!ret.isEmpty());
@@ -513,7 +796,8 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
      * @throws Exception If failed.
      */
     public void testDecrement() throws Exception {
-        String ret = content(F.asMap("cmd", "decr", "key", "decrKey", "init", "15", "delta", "10"));
+        String ret = content(F.asMap("cmd", GridRestCommand.ATOMIC_DECREMENT.key(),
+            "key", "decrKey", "init", "15", "delta", "10"));
 
         assertNotNull(ret);
         assertTrue(!ret.isEmpty());
@@ -522,7 +806,8 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
 
         assertEquals(5, grid(0).atomicLong("decrKey", 0, true).get());
 
-        ret = content(F.asMap("cmd", "decr", "key", "decrKey", "delta", "3"));
+        ret = content(F.asMap("cmd", GridRestCommand.ATOMIC_DECREMENT.key(),
+            "key", "decrKey", "delta", "3"));
 
         assertNotNull(ret);
         assertTrue(!ret.isEmpty());
@@ -540,7 +825,8 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
 
         assertEquals("casOldVal", jcache().localPeek("casKey", CachePeekMode.ONHEAP));
 
-        String ret = content(F.asMap("cmd", "cas", "key", "casKey", "val2", "casOldVal"));
+        String ret = content(F.asMap("cmd", GridRestCommand.CACHE_CAS.key(),
+            "key", "casKey", "val2", "casOldVal"));
 
         assertNotNull(ret);
         assertTrue(!ret.isEmpty());
@@ -558,7 +844,8 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
     public void testPutIfAbsent() throws Exception {
         assertNull(jcache().localPeek("casKey", CachePeekMode.ONHEAP));
 
-        String ret = content(F.asMap("cmd", "cas", "key", "casKey", "val1", "casNewVal"));
+        String ret = content(F.asMap("cmd", GridRestCommand.CACHE_CAS.key(),
+            "key", "casKey", "val1", "casNewVal"));
 
         assertNotNull(ret);
         assertTrue(!ret.isEmpty());
@@ -578,7 +865,7 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
 
         assertEquals("casVal", jcache().localPeek("casKey", CachePeekMode.ONHEAP));
 
-        String ret = content(F.asMap("cmd", "cas", "key", "casKey"));
+        String ret = content(F.asMap("cmd", GridRestCommand.CACHE_CAS.key(), "key", "casKey"));
 
         assertNotNull(ret);
         assertTrue(!ret.isEmpty());
@@ -594,7 +881,7 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
      * @throws Exception If failed.
      */
     public void testMetrics() throws Exception {
-        String ret = content(F.asMap("cmd", "cache"));
+        String ret = content(F.asMap("cmd", GridRestCommand.CACHE_METRICS.key()));
 
         assertNotNull(ret);
         assertTrue(!ret.isEmpty());
@@ -608,7 +895,7 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
      * @throws Exception If failed.
      */
     public void testTopology() throws Exception {
-        String ret = content(F.asMap("cmd", "top", "attr", "false", "mtr", "false"));
+        String ret = content(F.asMap("cmd", GridRestCommand.TOPOLOGY.key(), "attr", "false", "mtr", "false"));
 
         assertNotNull(ret);
         assertTrue(!ret.isEmpty());
@@ -622,7 +909,7 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
      * @throws Exception If failed.
      */
     public void testNode() throws Exception {
-        String ret = content(F.asMap("cmd", "node", "attr", "true", "mtr", "true", "id",
+        String ret = content(F.asMap("cmd", GridRestCommand.NODE.key(), "attr", "true", "mtr", "true", "id",
             grid(0).localNode().id().toString()));
 
         assertNotNull(ret);
@@ -632,7 +919,7 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
 
         jsonEquals(ret, pattern("\\{.+\\}", true));
 
-        ret = content(F.asMap("cmd", "node", "attr", "false", "mtr", "false", "ip", LOC_HOST));
+        ret = content(F.asMap("cmd", GridRestCommand.NODE.key(), "attr", "false", "mtr", "false", "ip", LOC_HOST));
 
         assertNotNull(ret);
         assertTrue(!ret.isEmpty());
@@ -641,7 +928,7 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
 
         jsonEquals(ret, pattern("\\{.+\\}", true));
 
-        ret = content(F.asMap("cmd", "node", "attr", "false", "mtr", "false", "ip", LOC_HOST, "id",
+        ret = content(F.asMap("cmd", GridRestCommand.NODE.key(), "attr", "false", "mtr", "false", "ip", LOC_HOST, "id",
             UUID.randomUUID().toString()));
 
         assertNotNull(ret);
@@ -660,7 +947,7 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
      * @throws Exception If failed.
      */
     public void testExe() throws Exception {
-        String ret = content(F.asMap("cmd", "exe"));
+        String ret = content(F.asMap("cmd", GridRestCommand.EXE.key()));
 
         assertNotNull(ret);
         assertTrue(!ret.isEmpty());
@@ -670,7 +957,7 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
         jsonEquals(ret, pattern("null", false));
 
         // Attempt to execute unknown task (UNKNOWN_TASK) will result in exception on server.
-        ret = content(F.asMap("cmd", "exe", "name", "UNKNOWN_TASK"));
+        ret = content(F.asMap("cmd", GridRestCommand.EXE.key(), "name", "UNKNOWN_TASK"));
 
         assertNotNull(ret);
         assertTrue(!ret.isEmpty());
@@ -682,7 +969,7 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
         grid(0).compute().localDeployTask(TestTask1.class, TestTask1.class.getClassLoader());
         grid(0).compute().localDeployTask(TestTask2.class, TestTask2.class.getClassLoader());
 
-        ret = content(F.asMap("cmd", "exe", "name", TestTask1.class.getName()));
+        ret = content(F.asMap("cmd", GridRestCommand.EXE.key(), "name", TestTask1.class.getName()));
 
         assertNotNull(ret);
         assertTrue(!ret.isEmpty());
@@ -691,7 +978,7 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
 
         jsonEquals(ret, pattern("\\{.+\\}", true));
 
-        ret = content(F.asMap("cmd", "exe", "name", TestTask2.class.getName()));
+        ret = content(F.asMap("cmd", GridRestCommand.EXE.key(), "name", TestTask2.class.getName()));
 
         assertNotNull(ret);
         assertTrue(!ret.isEmpty());
@@ -700,7 +987,7 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
 
         jsonEquals(ret, pattern("\\{.+" + TestTask2.RES + ".+\\}", true));
 
-        ret = content(F.asMap("cmd", "res"));
+        ret = content(F.asMap("cmd", GridRestCommand.RESULT.key()));
 
         assertNotNull(ret);
         assertTrue(!ret.isEmpty());
@@ -722,5 +1009,266 @@ abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestProcessorS
         jsonEquals(ret, stringPattern(".+", true));
     }
 
+    /**
+     * @throws Exception If failed.
+     */
+    public void testQueryArgs() throws Exception {
+        String qry = "salary > ? and salary <= ?";
+
+        Map<String, String> params = new HashMap<>();
+        params.put("cmd", GridRestCommand.EXECUTE_SQL_QUERY.key());
+        params.put("type", "Person");
+        params.put("psz", "10");
+        params.put("cacheName", "person");
+        params.put("qry", URLEncoder.encode(qry));
+        params.put("arg1", "1000");
+        params.put("arg2", "2000");
+
+        String ret = content(params);
+
+        assertNotNull(ret);
+        assertTrue(!ret.isEmpty());
+
+        JSONObject json = JSONObject.fromObject(ret);
+
+        List items = (List)((Map)json.get("response")).get("items");
+
+        assertEquals(2, items.size());
+
+        assertFalse(queryCursorFound());
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testQuery() throws Exception {
+        grid(0).cache(null).put("1", "1");
+        grid(0).cache(null).put("2", "2");
+        grid(0).cache(null).put("3", "3");
+
+        Map<String, String> params = new HashMap<>();
+        params.put("cmd", GridRestCommand.EXECUTE_SQL_QUERY.key());
+        params.put("type", "String");
+        params.put("psz", "1");
+        params.put("qry", URLEncoder.encode("select * from String"));
+
+        String ret = content(params);
+
+        assertNotNull(ret);
+        assertTrue(!ret.isEmpty());
+
+        JSONObject json = JSONObject.fromObject(ret);
+
+        Integer qryId = (Integer)((Map)json.get("response")).get("queryId");
+
+        assertNotNull(qryId);
+
+        ret = content(F.asMap("cmd", GridRestCommand.FETCH_SQL_QUERY.key(),
+            "psz", "1", "qryId", String.valueOf(qryId)));
+
+        json = JSONObject.fromObject(ret);
+
+        Integer qryId0 = (Integer)((Map)json.get("response")).get("queryId");
+
+        Boolean last = (Boolean)((Map)json.get("response")).get("last");
+
+        assertEquals(qryId0, qryId);
+        assertFalse(last);
+
+        ret = content(F.asMap("cmd", GridRestCommand.FETCH_SQL_QUERY.key(),
+            "psz", "1", "qryId", String.valueOf(qryId)));
+
+        json = JSONObject.fromObject(ret);
+
+        qryId0 = (Integer)((Map)json.get("response")).get("queryId");
+
+        last = (Boolean)((Map)json.get("response")).get("last");
+
+        assertEquals(qryId0, qryId);
+        assertTrue(last);
+
+        assertFalse(queryCursorFound());
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testSqlFieldsQuery() throws Exception {
+        String qry = "select concat(firstName, ' ', lastName) from Person";
+
+        Map<String, String> params = new HashMap<>();
+        params.put("cmd", GridRestCommand.EXECUTE_SQL_FIELDS_QUERY.key());
+        params.put("psz", "10");
+        params.put("cacheName", "person");
+        params.put("qry", URLEncoder.encode(qry));
+
+        String ret = content(params);
+
+        assertNotNull(ret);
+        assertTrue(!ret.isEmpty());
+
+        JSONObject json = JSONObject.fromObject(ret);
+
+        List items = (List)((Map)json.get("response")).get("items");
+
+        assertEquals(4, items.size());
+
+        assertFalse(queryCursorFound());
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testQueryClose() throws Exception {
+        String qry = "salary > ? and salary <= ?";
+
+        Map<String, String> params = new HashMap<>();
+        params.put("cmd", GridRestCommand.EXECUTE_SQL_QUERY.key());
+        params.put("type", "Person");
+        params.put("psz", "1");
+        params.put("cacheName", "person");
+        params.put("qry", URLEncoder.encode(qry));
+        params.put("arg1", "1000");
+        params.put("arg2", "2000");
+
+        String ret = content(params);
+
+        assertNotNull(ret);
+        assertTrue(!ret.isEmpty());
+
+        JSONObject json = JSONObject.fromObject(ret);
+
+        List items = (List)((Map)json.get("response")).get("items");
+
+        assertEquals(1, items.size());
+
+        assertTrue(queryCursorFound());
+
+        Integer qryId = (Integer)((Map)json.get("response")).get("queryId");
+
+        assertNotNull(qryId);
+
+        ret = content(F.asMap("cmd", GridRestCommand.CLOSE_SQL_QUERY.key(),
+            "cacheName", "person", "qryId", String.valueOf(qryId)));
+
+        assertNotNull(ret);
+        assertTrue(!ret.isEmpty());
+
+        assertFalse(queryCursorFound());
+    }
+
     protected abstract String signature() throws Exception;
+
+    /**
+     * @return True if any query cursor is available.
+     */
+    private boolean queryCursorFound() {
+        boolean found = false;
+
+        for (int i = 0; i < GRID_CNT; ++i) {
+            Map<GridRestCommand, GridRestCommandHandler> handlers =
+                GridTestUtils.getFieldValue(grid(i).context().rest(), "handlers");
+
+            GridRestCommandHandler qryHnd = handlers.get(GridRestCommand.CLOSE_SQL_QUERY);
+
+            ConcurrentHashMap<Long, Iterator> its = GridTestUtils.getFieldValue(qryHnd, "qryCurs");
+
+            found |= its.size() != 0;
+        }
+
+        return found;
+    }
+
+    /**
+     * Init cache.
+     */
+    private void initCache() {
+        CacheConfiguration<Integer, Person> personCacheCfg = new CacheConfiguration<>("person");
+        personCacheCfg.setIndexedTypes(Integer.class, Person.class);
+
+        IgniteCache<Integer, Person> personCache = grid(0).getOrCreateCache(personCacheCfg);
+
+        personCache.clear();
+
+        Person p1 = new Person("John", "Doe", 2000);
+        Person p2 = new Person("Jane", "Doe", 1000);
+        Person p3 = new Person("John", "Smith", 1000);
+        Person p4 = new Person("Jane", "Smith", 2000);
+
+        personCache.put(p1.getId(), p1);
+        personCache.put(p2.getId(), p2);
+        personCache.put(p3.getId(), p3);
+        personCache.put(p4.getId(), p4);
+
+        SqlQuery<Integer, Person> qry = new SqlQuery<>(Person.class, "salary > ? and salary <= ?");
+
+        qry.setArgs(1000, 2000);
+
+        assertEquals(2, personCache.query(qry).getAll().size());
+    }
+
+    /**
+     * Person class.
+     */
+    public static class Person implements Serializable {
+        /** Person id. */
+        private static int PERSON_ID = 0;
+
+        /** Person ID (indexed). */
+        @QuerySqlField(index = true)
+        private Integer id;
+
+        /** First name (not-indexed). */
+        @QuerySqlField
+        private String firstName;
+
+        /** Last name (not indexed). */
+        @QuerySqlField
+        private String lastName;
+
+        /** Salary (indexed). */
+        @QuerySqlField(index = true)
+        private double salary;
+
+        /**
+         * @param firstName First name.
+         * @param lastName Last name.
+         * @param salary Salary.
+         */
+        Person(String firstName, String lastName, double salary) {
+            id = PERSON_ID++;
+
+            this.firstName = firstName;
+            this.lastName = lastName;
+            this.salary = salary;
+        }
+
+        /**
+         * @return First name.
+         */
+        public String getFirstName() {
+            return firstName;
+        }
+
+        /**
+         * @return Last name.
+         */
+        public String getLastName() {
+            return lastName;
+        }
+        /**
+         * @return Salary.
+         */
+        public double getSalary() {
+
+            return salary;
+        }
+
+        /**
+         * @return Id.
+         */
+        public Integer getId() {
+            return id;
+        }
+    }
 }

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/9bfc1842/modules/core/src/main/java/org/apache/ignite/internal/IgniteKernal.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/IgniteKernal.java b/modules/core/src/main/java/org/apache/ignite/internal/IgniteKernal.java
index daa9204..2d8e88f 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/IgniteKernal.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/IgniteKernal.java
@@ -2448,22 +2448,28 @@ public class IgniteKernal implements IgniteEx, IgniteMXBean, Externalizable {
 
     /** {@inheritDoc} */
     @Override public void destroyCache(String cacheName) {
-        guard();
-
-        IgniteInternalFuture<?> stopFut;
+        IgniteInternalFuture stopFut = destroyCacheAsync(cacheName);
 
         try {
-            stopFut = ctx.cache().dynamicDestroyCache(cacheName);
+            stopFut.get();
         }
-        finally {
-            unguard();
+        catch (IgniteCheckedException e) {
+            throw CU.convertToCacheException(e);
         }
+    }
+
+    /**
+     * @param cacheName Cache name.
+     * @return Ignite future.
+     */
+    public IgniteInternalFuture<?> destroyCacheAsync(String cacheName) {
+        guard();
 
         try {
-            stopFut.get();
+            return ctx.cache().dynamicDestroyCache(cacheName);
         }
-        catch (IgniteCheckedException e) {
-            throw CU.convertToCacheException(e);
+        finally {
+            unguard();
         }
     }
 
@@ -2485,6 +2491,24 @@ public class IgniteKernal implements IgniteEx, IgniteMXBean, Externalizable {
         }
     }
 
+    /**
+     * @param cacheName Cache name.
+     * @return Future that will be completed when cache is deployed.
+     */
+    public IgniteInternalFuture<?> getOrCreateCacheAsync(String cacheName) {
+        guard();
+
+        try {
+            if (ctx.cache().cache(cacheName) == null)
+                return ctx.cache().getOrCreateFromTemplate(cacheName);
+
+            return new GridFinishedFuture<>();
+        }
+        finally {
+            unguard();
+        }
+    }
+
     /** {@inheritDoc} */
     @Override public <K, V> void addCacheConfiguration(CacheConfiguration<K, V> cacheCfg) {
         A.notNull(cacheCfg, "cacheCfg");

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/9bfc1842/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheAdapter.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheAdapter.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheAdapter.java
index 2e5fe8c..94bcc93 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheAdapter.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheAdapter.java
@@ -2279,7 +2279,7 @@ public abstract class GridCacheAdapter<K, V> implements IgniteInternalCache<K, V
      * @return Put future.
      */
     public IgniteInternalFuture<Boolean> putAsync(K key, V val,
-                                                  @Nullable CacheEntryPredicate... filter) {
+        @Nullable CacheEntryPredicate... filter) {
         final boolean statsEnabled = ctx.config().isStatisticsEnabled();
 
         final long start = statsEnabled ? System.nanoTime() : 0L;

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/9bfc1842/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/GridRestCommand.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/GridRestCommand.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/GridRestCommand.java
index 62732f0..76a33b1 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/GridRestCommand.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/GridRestCommand.java
@@ -33,12 +33,27 @@ public enum GridRestCommand {
     /** Get cached value. */
     CACHE_GET("get"),
 
+    /** Contains cached value. */
+    CACHE_CONTAINS_KEY("conkey"),
+
+    /** Contains cached values. */
+    CACHE_CONTAINS_KEYS("conkeys"),
+
     /** Get several cached values. */
     CACHE_GET_ALL("getall"),
 
+    /** Store value in cache and return previous value. */
+    CACHE_GET_AND_PUT("getput"),
+
+    /** Store value in cache and return previous value. */
+    CACHE_GET_AND_PUT_IF_ABSENT("getputifabs"),
+
     /** Store value in cache. */
     CACHE_PUT("put"),
 
+    /** Store value in cache. */
+    CACHE_PUT_IF_ABSENT("putifabs"),
+
     /** Store value in cache if it doesn't exist. */
     CACHE_ADD("add"),
 
@@ -48,12 +63,24 @@ public enum GridRestCommand {
     /** Remove value from cache. */
     CACHE_REMOVE("rmv"),
 
+    /** Remove value from cache. */
+    CACHE_REMOVE_VALUE("rmvval"),
+
+    /** Remove value from cache. */
+    CACHE_GET_AND_REMOVE("getrmv"),
+
     /** Remove several values from cache. */
     CACHE_REMOVE_ALL("rmvall"),
 
     /** Replace cache value only if there is currently a mapping for it. */
     CACHE_REPLACE("rep"),
 
+    /** Replace cache value only if there is currently a mapping for it. */
+    CACHE_REPLACE_VALUE("repval"),
+
+    /** Replace cache value only if there is currently a mapping for it. */
+    CACHE_GET_AND_REPLACE("getrep"),
+
     /** Compare and set. */
     CACHE_CAS("cas"),
 
@@ -66,6 +93,9 @@ public enum GridRestCommand {
     /** Cache metrics. */
     CACHE_METRICS("cache"),
 
+    /** Cache size. */
+    CACHE_SIZE("size"),
+
     /** Increment. */
     ATOMIC_INCREMENT("incr"),
 
@@ -87,6 +117,9 @@ public enum GridRestCommand {
     /** Version. */
     VERSION("version"),
 
+    /** Name. */
+    NAME("name"),
+
     /** Log. */
     LOG("log"),
 
@@ -94,7 +127,25 @@ public enum GridRestCommand {
     NOOP("noop"),
 
     /** Quit. */
-    QUIT("quit");
+    QUIT("quit"),
+
+    /** Get or create cache. */
+    GET_OR_CREATE_CACHE("getorcreate"),
+
+    /** Stops dynamically started cache. */
+    DESTROY_CACHE("destcache"),
+
+    /** Execute sql query. */
+    EXECUTE_SQL_QUERY("qryexe"),
+
+    /** Execute sql fields query. */
+    EXECUTE_SQL_FIELDS_QUERY("qryfldexe"),
+
+    /** Fetch query results. */
+    FETCH_SQL_QUERY("qryfetch"),
+
+    /** Close query. */
+    CLOSE_SQL_QUERY("qrycls");
 
     /** Enum values. */
     private static final GridRestCommand[] VALS = values();

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/9bfc1842/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/GridRestProcessor.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/GridRestProcessor.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/GridRestProcessor.java
index 2d1d802..bb796d2 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/GridRestProcessor.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/GridRestProcessor.java
@@ -25,6 +25,7 @@ import org.apache.ignite.internal.processors.rest.client.message.*;
 import org.apache.ignite.internal.processors.rest.handlers.*;
 import org.apache.ignite.internal.processors.rest.handlers.cache.*;
 import org.apache.ignite.internal.processors.rest.handlers.datastructures.*;
+import org.apache.ignite.internal.processors.rest.handlers.query.*;
 import org.apache.ignite.internal.processors.rest.handlers.task.*;
 import org.apache.ignite.internal.processors.rest.handlers.top.*;
 import org.apache.ignite.internal.processors.rest.handlers.version.*;
@@ -254,6 +255,7 @@ public class GridRestProcessor extends GridProcessorAdapter {
             addHandler(new GridTopologyCommandHandler(ctx));
             addHandler(new GridVersionCommandHandler(ctx));
             addHandler(new DataStructuresCommandHandler(ctx));
+            addHandler(new QueryCommandHandler(ctx));
 
             // Start protocols.
             startTcpProtocol();
@@ -384,6 +386,8 @@ public class GridRestProcessor extends GridProcessorAdapter {
 
         if (interceptor != null && res.getResponse() != null) {
             switch (req.command()) {
+                case CACHE_CONTAINS_KEYS:
+                case CACHE_CONTAINS_KEY:
                 case CACHE_GET:
                 case CACHE_GET_ALL:
                 case CACHE_PUT:
@@ -527,12 +531,23 @@ public class GridRestProcessor extends GridProcessorAdapter {
 
         switch (req.command()) {
             case CACHE_GET:
+            case CACHE_CONTAINS_KEY:
+            case CACHE_CONTAINS_KEYS:
             case CACHE_GET_ALL:
                 perm = SecurityPermission.CACHE_READ;
                 name = ((GridRestCacheRequest)req).cacheName();
 
                 break;
 
+            case EXECUTE_SQL_QUERY:
+            case EXECUTE_SQL_FIELDS_QUERY:
+            case CLOSE_SQL_QUERY:
+            case FETCH_SQL_QUERY:
+                perm = SecurityPermission.CACHE_READ;
+                name = ((RestSqlQueryRequest)req).cacheName();
+
+                break;
+
             case CACHE_PUT:
             case CACHE_ADD:
             case CACHE_PUT_ALL:
@@ -540,6 +555,11 @@ public class GridRestProcessor extends GridProcessorAdapter {
             case CACHE_CAS:
             case CACHE_APPEND:
             case CACHE_PREPEND:
+            case CACHE_GET_AND_PUT:
+            case CACHE_GET_AND_REPLACE:
+            case CACHE_GET_AND_PUT_IF_ABSENT:
+            case CACHE_PUT_IF_ABSENT:
+            case CACHE_REPLACE_VALUE:
                 perm = SecurityPermission.CACHE_PUT;
                 name = ((GridRestCacheRequest)req).cacheName();
 
@@ -547,6 +567,8 @@ public class GridRestProcessor extends GridProcessorAdapter {
 
             case CACHE_REMOVE:
             case CACHE_REMOVE_ALL:
+            case CACHE_GET_AND_REMOVE:
+            case CACHE_REMOVE_VALUE:
                 perm = SecurityPermission.CACHE_REMOVE;
                 name = ((GridRestCacheRequest)req).cacheName();
 
@@ -559,7 +581,15 @@ public class GridRestProcessor extends GridProcessorAdapter {
 
                 break;
 
+            case GET_OR_CREATE_CACHE:
+            case DESTROY_CACHE:
+                perm = SecurityPermission.ADMIN_CACHE;
+                name = ((GridRestCacheRequest)req).cacheName();
+
+                break;
+
             case CACHE_METRICS:
+            case CACHE_SIZE:
             case TOPOLOGY:
             case NODE:
             case VERSION:
@@ -567,6 +597,8 @@ public class GridRestProcessor extends GridProcessorAdapter {
             case QUIT:
             case ATOMIC_INCREMENT:
             case ATOMIC_DECREMENT:
+            case NAME:
+            case LOG:
                 break;
 
             default:

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/9bfc1842/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/handlers/cache/GridCacheCommandHandler.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/handlers/cache/GridCacheCommandHandler.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/handlers/cache/GridCacheCommandHandler.java
index 1f24023..64f595d 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/handlers/cache/GridCacheCommandHandler.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/handlers/cache/GridCacheCommandHandler.java
@@ -53,26 +53,46 @@ import static org.apache.ignite.transactions.TransactionIsolation.*;
 public class GridCacheCommandHandler extends GridRestCommandHandlerAdapter {
     /** Supported commands. */
     private static final Collection<GridRestCommand> SUPPORTED_COMMANDS = U.sealList(
+        DESTROY_CACHE,
+        GET_OR_CREATE_CACHE,
+        CACHE_CONTAINS_KEYS,
+        CACHE_CONTAINS_KEY,
         CACHE_GET,
+        CACHE_GET_AND_PUT,
+        CACHE_GET_AND_REPLACE,
+        CACHE_GET_AND_PUT_IF_ABSENT,
+        CACHE_PUT_IF_ABSENT,
         CACHE_GET_ALL,
         CACHE_PUT,
         CACHE_ADD,
         CACHE_PUT_ALL,
         CACHE_REMOVE,
+        CACHE_REMOVE_VALUE,
+        CACHE_REPLACE_VALUE,
+        CACHE_GET_AND_REMOVE,
         CACHE_REMOVE_ALL,
         CACHE_REPLACE,
         CACHE_CAS,
         CACHE_APPEND,
         CACHE_PREPEND,
-        CACHE_METRICS
+        CACHE_METRICS,
+        CACHE_SIZE
     );
 
     /** Requests with required parameter {@code key}. */
     private static final EnumSet<GridRestCommand> KEY_REQUIRED_REQUESTS = EnumSet.of(
+        CACHE_CONTAINS_KEY,
         CACHE_GET,
+        CACHE_GET_AND_PUT,
+        CACHE_GET_AND_REPLACE,
+        CACHE_GET_AND_PUT_IF_ABSENT,
+        CACHE_PUT_IF_ABSENT,
         CACHE_PUT,
         CACHE_ADD,
         CACHE_REMOVE,
+        CACHE_REMOVE_VALUE,
+        CACHE_REPLACE_VALUE,
+        CACHE_GET_AND_REMOVE,
         CACHE_REPLACE,
         ATOMIC_INCREMENT,
         ATOMIC_DECREMENT,
@@ -137,6 +157,44 @@ public class GridCacheCommandHandler extends GridRestCommandHandlerAdapter {
             IgniteInternalFuture<GridRestResponse> fut;
 
             switch (cmd) {
+                case DESTROY_CACHE: {
+                    fut = ((IgniteKernal)ctx.grid()).destroyCacheAsync(cacheName).chain(
+                        new CX1<IgniteInternalFuture<?>, GridRestResponse>() {
+                            @Override public GridRestResponse applyx(IgniteInternalFuture<?> f)
+                                throws IgniteCheckedException {
+                                return new GridRestResponse(f.get());
+                            }
+                        });
+
+                    break;
+                }
+
+                case GET_OR_CREATE_CACHE: {
+                    fut = ((IgniteKernal)ctx.grid()).getOrCreateCacheAsync(cacheName).chain(
+                        new CX1<IgniteInternalFuture<?>, GridRestResponse>() {
+                            @Override public GridRestResponse applyx(IgniteInternalFuture<?> f)
+                                throws IgniteCheckedException {
+                                return new GridRestResponse(f.get());
+                            }
+                        });
+
+                    break;
+                }
+
+                case CACHE_CONTAINS_KEYS: {
+                    fut = executeCommand(req.destinationId(), req.clientId(), cacheName, skipStore, key,
+                        new ContainsKeysCommand(getKeys(req0)));
+
+                    break;
+                }
+
+                case CACHE_CONTAINS_KEY: {
+                    fut = executeCommand(req.destinationId(), req.clientId(), cacheName, skipStore, key,
+                        new ContainsKeyCommand(key));
+
+                    break;
+                }
+
                 case CACHE_GET: {
                     fut = executeCommand(req.destinationId(), req.clientId(), cacheName, skipStore, key,
                         new GetCommand(key));
@@ -144,48 +202,51 @@ public class GridCacheCommandHandler extends GridRestCommandHandlerAdapter {
                     break;
                 }
 
-                case CACHE_GET_ALL: {
-                    Set<Object> keys = req0.values().keySet();
+                case CACHE_GET_AND_PUT: {
+                    fut = executeCommand(req.destinationId(), req.clientId(), cacheName, skipStore, key,
+                        new GetAndPutCommand(key, getValue(req0)));
 
-                    if (F.isEmpty(keys))
-                        throw new IgniteCheckedException(GridRestCommandHandlerAdapter.missingParameter("keys"));
+                    break;
+                }
 
-                    // HashSet wrapping for correct serialization
-                    HashSet<Object> keys0 = new HashSet<>();
+                case CACHE_GET_AND_REPLACE: {
+                    fut = executeCommand(req.destinationId(), req.clientId(), cacheName, skipStore, key,
+                        new GetAndReplaceCommand(key, getValue(req0)));
 
-                    for (Object getKey : keys) {
-                        if (getKey == null)
-                            throw new IgniteCheckedException("Failing getAll operation (null keys are not allowed).");
+                    break;
+                }
 
-                        keys0.add(getKey);
-                    }
+                case CACHE_GET_AND_PUT_IF_ABSENT: {
+                    fut = executeCommand(req.destinationId(), req.clientId(), cacheName, skipStore, key,
+                        new GetAndPutIfAbsentCommand(key, getValue(req0)));
+
+                    break;
+                }
 
+                case CACHE_PUT_IF_ABSENT: {
                     fut = executeCommand(req.destinationId(), req.clientId(), cacheName, skipStore, key,
-                        new GetAllCommand(keys0));
+                        new PutIfAbsentCommand(key, getValue(req0)));
 
                     break;
                 }
 
-                case CACHE_PUT: {
-                    final Object val = req0.value();
+                case CACHE_GET_ALL: {
+                    fut = executeCommand(req.destinationId(), req.clientId(), cacheName, skipStore, key,
+                        new GetAllCommand(getKeys(req0)));
 
-                    if (val == null)
-                        throw new IgniteCheckedException(GridRestCommandHandlerAdapter.missingParameter("val"));
+                    break;
+                }
 
+                case CACHE_PUT: {
                     fut = executeCommand(req.destinationId(), req.clientId(), cacheName, skipStore, key, new
-                        PutCommand(key, ttl, val));
+                        PutCommand(key, ttl, getValue(req0)));
 
                     break;
                 }
 
                 case CACHE_ADD: {
-                    final Object val = req0.value();
-
-                    if (val == null)
-                        throw new IgniteCheckedException(GridRestCommandHandlerAdapter.missingParameter("val"));
-
                     fut = executeCommand(req.destinationId(), req.clientId(), cacheName, skipStore, key,
-                        new AddCommand(key, ttl, val));
+                        new AddCommand(key, ttl, getValue(req0)));
 
                     break;
                 }
@@ -220,6 +281,27 @@ public class GridCacheCommandHandler extends GridRestCommandHandlerAdapter {
                     break;
                 }
 
+                case CACHE_REMOVE_VALUE: {
+                    fut = executeCommand(req.destinationId(), req.clientId(), cacheName, skipStore, key,
+                        new RemoveValueCommand(key, getValue(req0)));
+
+                    break;
+                }
+
+                case CACHE_REPLACE_VALUE: {
+                    fut = executeCommand(req.destinationId(), req.clientId(), cacheName, skipStore, key,
+                        new ReplaceValueCommand(key, getValue(req0), req0.value2()));
+
+                    break;
+                }
+
+                case CACHE_GET_AND_REMOVE: {
+                    fut = executeCommand(req.destinationId(), req.clientId(), cacheName, skipStore, key,
+                        new GetAndRemoveCommand(key));
+
+                    break;
+                }
+
                 case CACHE_REMOVE_ALL: {
                     Map<Object, Object> map = req0.values();
 
@@ -274,6 +356,12 @@ public class GridCacheCommandHandler extends GridRestCommandHandlerAdapter {
                     break;
                 }
 
+                case CACHE_SIZE: {
+                    fut = executeCommand(req.destinationId(), req.clientId(), cacheName, key, new SizeCommand());
+
+                    break;
+                }
+
                 default:
                     throw new IllegalArgumentException("Invalid command for cache handler: " + req);
             }
@@ -297,6 +385,44 @@ public class GridCacheCommandHandler extends GridRestCommandHandlerAdapter {
     }
 
     /**
+     * @param req Request.
+     * @return Request keys.
+     * @throws IgniteCheckedException If incorrect keys are presented.
+     */
+    private Set<Object> getKeys(GridRestCacheRequest req) throws IgniteCheckedException {
+        Set<Object> keys = req.values().keySet();
+
+        if (F.isEmpty(keys))
+            throw new IgniteCheckedException(GridRestCommandHandlerAdapter.missingParameter("keys"));
+
+        // HashSet wrapping for correct serialization
+        HashSet<Object> keys0 = new HashSet<>();
+
+        for (Object getKey : keys) {
+            if (getKey == null)
+                throw new IgniteCheckedException("Failing operation (null keys are not allowed).");
+
+            keys0.add(getKey);
+        }
+
+        return keys0;
+    }
+
+    /**
+     * @param req Request.
+     * @return Request value.
+     * @throws IgniteCheckedException If incorrect keys are presented.
+     */
+    private Object getValue(GridRestCacheRequest req) throws IgniteCheckedException {
+        final Object val = req.value();
+
+        if (val == null)
+            throw new IgniteCheckedException(GridRestCommandHandlerAdapter.missingParameter("val"));
+
+        return val;
+    }
+
+    /**
      * Executes command on flagged cache projection. Checks {@code destId} to find
      * if command could be performed locally or routed to a remote node.
      *
@@ -702,6 +828,48 @@ public class GridCacheCommandHandler extends GridRestCommandHandlerAdapter {
     }
 
     /** */
+    private static class ContainsKeyCommand extends CacheProjectionCommand {
+        /** */
+        private static final long serialVersionUID = 0L;
+
+        /** */
+        private final Object key;
+
+        /**
+         * @param key Key.
+         */
+        ContainsKeyCommand(Object key) {
+            this.key = key;
+        }
+
+        /** {@inheritDoc} */
+        @Override public IgniteInternalFuture<?> applyx(IgniteInternalCache<Object, Object> c, GridKernalContext ctx) {
+            return c.containsKeyAsync(key);
+        }
+    }
+
+    /** */
+    private static class ContainsKeysCommand extends CacheProjectionCommand {
+        /** */
+        private static final long serialVersionUID = 0L;
+
+        /** */
+        private final Collection<Object> keys;
+
+        /**
+         * @param keys Keys.
+         */
+        ContainsKeysCommand(Collection<Object> keys) {
+            this.keys = keys;
+        }
+
+        /** {@inheritDoc} */
+        @Override public IgniteInternalFuture<?> applyx(IgniteInternalCache<Object, Object> c, GridKernalContext ctx) {
+            return c.containsKeysAsync(keys);
+        }
+    }
+
+    /** */
     private static class GetCommand extends CacheProjectionCommand {
         /** */
         private static final long serialVersionUID = 0L;
@@ -723,6 +891,113 @@ public class GridCacheCommandHandler extends GridRestCommandHandlerAdapter {
     }
 
     /** */
+    private static class GetAndPutCommand extends CacheProjectionCommand {
+        /** */
+        private static final long serialVersionUID = 0L;
+
+        /** Key. */
+        protected final Object key;
+
+        /** Value.*/
+        protected final Object val;
+
+        /**
+         * @param key Key.
+         * @param val Value.
+         */
+        GetAndPutCommand(Object key, Object val) {
+            this.key = key;
+            this.val = val;
+        }
+
+        /** {@inheritDoc} */
+        @Override public IgniteInternalFuture<?> applyx(IgniteInternalCache<Object, Object> c, GridKernalContext ctx) {
+            return c.getAndPutAsync(key, val);
+        }
+    }
+
+    /** */
+    private static class GetAndReplaceCommand extends GetAndPutCommand {
+        /** */
+        private static final long serialVersionUID = 0L;
+
+        /**
+         * @param key Key.
+         * @param val Value.
+         */
+        GetAndReplaceCommand(Object key, Object val) {
+            super(key, val);
+        }
+
+        /** {@inheritDoc} */
+        @Override public IgniteInternalFuture<?> applyx(IgniteInternalCache<Object, Object> c, GridKernalContext ctx) {
+            return c.getAndReplaceAsync(key, val);
+        }
+    }
+
+    /** */
+    private static class ReplaceValueCommand extends GetAndReplaceCommand {
+        /** */
+        private static final long serialVersionUID = 0L;
+
+        /** */
+        private final Object oldVal;
+
+        /**
+         * @param key Key.
+         * @param val Value.
+         * @param oldVal Old value.
+         */
+        ReplaceValueCommand(Object key, Object val, Object oldVal) {
+            super(key, val);
+            this.oldVal = oldVal;
+        }
+
+        /** {@inheritDoc} */
+        @Override public IgniteInternalFuture<?> applyx(IgniteInternalCache<Object, Object> c, GridKernalContext ctx) {
+            return c.replaceAsync(key, oldVal, val);
+        }
+    }
+
+    /** */
+    private static class GetAndPutIfAbsentCommand extends GetAndPutCommand {
+        /** */
+        private static final long serialVersionUID = 0L;
+
+        /**
+         * @param key Key.
+         * @param val Value.
+         */
+        GetAndPutIfAbsentCommand(Object key, Object val) {
+            super(key, val);
+        }
+
+        /** {@inheritDoc} */
+        @Override public IgniteInternalFuture<?> applyx(IgniteInternalCache<Object, Object> c, GridKernalContext ctx) {
+            return c.getAndPutIfAbsentAsync(key, val);
+        }
+    }
+
+    /** */
+    private static class PutIfAbsentCommand extends GetAndPutCommand {
+        /** */
+        private static final long serialVersionUID = 0L;
+
+        /**
+         * @param key Key.
+         * @param val Value.
+         */
+        PutIfAbsentCommand(Object key, Object val) {
+            super(key, val);
+        }
+
+        /** {@inheritDoc} */
+        @Override public IgniteInternalFuture<?> applyx(IgniteInternalCache<Object, Object> c, GridKernalContext ctx) {
+            return c.putIfAbsentAsync(key, val);
+        }
+    }
+
+    /** */
     private static class GetAllCommand extends CacheProjectionCommand {
         /** */
         private static final long serialVersionUID = 0L;
@@ -770,7 +1045,7 @@ public class GridCacheCommandHandler extends GridRestCommandHandlerAdapter {
         private static final long serialVersionUID = 0L;
 
         /** */
-        private final Object key;
+        protected final Object key;
 
         /**
          * @param key Key.
@@ -786,6 +1061,43 @@ public class GridCacheCommandHandler extends GridRestCommandHandlerAdapter {
     }
 
     /** */
+    private static class RemoveValueCommand extends GetAndPutCommand {
+        /** */
+        private static final long serialVersionUID = 0L;
+
+        /**
+         * @param key Key.
+         * @param val Value.
+         */
+        RemoveValueCommand(Object key, Object val) {
+            super(key, val);
+        }
+
+        /** {@inheritDoc} */
+        @Override public IgniteInternalFuture<?> applyx(IgniteInternalCache<Object, Object> c, GridKernalContext ctx) {
+            return c.removeAsync(key, val);
+        }
+    }
+
+    /** */
+    private static class GetAndRemoveCommand extends RemoveCommand {
+        /** */
+        private static final long serialVersionUID = 0L;
+
+        /**
+         * @param key Key.
+         */
+        GetAndRemoveCommand(Object key) {
+            super(key);
+        }
+
+        /** {@inheritDoc} */
+        @Override public IgniteInternalFuture<?> applyx(IgniteInternalCache<Object, Object> c, GridKernalContext ctx) {
+            return c.getAndRemoveAsync(key);
+        }
+    }
+
+    /** */
     private static class RemoveAllCommand extends CacheProjectionCommand {
         /** */
         private static final long serialVersionUID = 0L;
@@ -1025,4 +1337,15 @@ public class GridCacheCommandHandler extends GridRestCommandHandlerAdapter {
             );
         }
     }
+
+    /** */
+    private static class SizeCommand extends CacheCommand {
+        /** */
+        private static final long serialVersionUID = 0L;
+
+        /** {@inheritDoc} */
+        @Override public IgniteInternalFuture<?> applyx(IgniteInternalCache<Object, Object> c, GridKernalContext ctx) {
+            return c.sizeAsync(new CachePeekMode[]{CachePeekMode.PRIMARY});
+        }
+    }
 }

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/9bfc1842/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/handlers/query/CacheQueryResult.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/handlers/query/CacheQueryResult.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/handlers/query/CacheQueryResult.java
new file mode 100644
index 0000000..3e49576
--- /dev/null
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/handlers/query/CacheQueryResult.java
@@ -0,0 +1,101 @@
+/*
+ * 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.rest.handlers.query;
+
+import org.apache.ignite.internal.util.typedef.internal.*;
+
+import java.io.*;
+import java.util.*;
+
+/**
+ * Client query result.
+ */
+public class CacheQueryResult implements Externalizable {
+    /** */
+    private static final long serialVersionUID = 0L;
+
+    /** Query ID. */
+    private long qryId;
+
+    /** Result items. */
+    private Collection<?> items;
+
+    /** Last flag. */
+    private boolean last;
+
+    /**
+     * @return Query ID.
+     */
+    public long getQueryId() {
+        return qryId;
+    }
+
+    /**
+     * @param qryId Query ID.
+     */
+    public void setQueryId(long qryId) {
+        this.qryId = qryId;
+    }
+
+    /**
+     * @return Items.
+     */
+    public Collection<?> getItems() {
+        return items;
+    }
+
+    /**
+     * @param items Items.
+     */
+    public void setItems(Collection<?> items) {
+        this.items = items;
+    }
+
+    /**
+     * @return Last flag.
+     */
+    public boolean getLast() {
+        return last;
+    }
+
+    /**
+     * @param last Last flag.
+     */
+    public void setLast(boolean last) {
+        this.last = last;
+    }
+
+    /** {@inheritDoc} */
+    @Override public String toString() {
+        return S.toString(CacheQueryResult.class, this);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeExternal(ObjectOutput out) throws IOException {
+        out.writeBoolean(last);
+        out.writeLong(qryId);
+        U.writeCollection(out, items);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void readExternal(ObjectInput in) throws IOException, ClassNotFoundException {
+        last = in.readBoolean();
+        qryId = in.readLong();
+        items = U.readCollection(in);
+    }
+}

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/9bfc1842/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/handlers/query/QueryCommandHandler.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/handlers/query/QueryCommandHandler.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/handlers/query/QueryCommandHandler.java
new file mode 100644
index 0000000..defca37
--- /dev/null
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/handlers/query/QueryCommandHandler.java
@@ -0,0 +1,271 @@
+/*
+ * 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.rest.handlers.query;
+
+import org.apache.ignite.*;
+import org.apache.ignite.cache.query.*;
+import org.apache.ignite.internal.*;
+import org.apache.ignite.internal.processors.rest.*;
+import org.apache.ignite.internal.processors.rest.handlers.*;
+import org.apache.ignite.internal.processors.rest.request.*;
+import org.apache.ignite.internal.util.future.*;
+import org.apache.ignite.internal.util.typedef.internal.*;
+import org.apache.ignite.lang.*;
+
+import java.util.*;
+import java.util.concurrent.*;
+import java.util.concurrent.atomic.*;
+
+import static org.apache.ignite.internal.processors.rest.GridRestCommand.*;
+
+/**
+ * Query command handler.
+ */
+public class QueryCommandHandler extends GridRestCommandHandlerAdapter {
+    /** Supported commands. */
+    private static final Collection<GridRestCommand> SUPPORTED_COMMANDS = U.sealList(EXECUTE_SQL_QUERY,
+        EXECUTE_SQL_FIELDS_QUERY,
+        FETCH_SQL_QUERY,
+        CLOSE_SQL_QUERY);
+
+    /** Query ID sequence. */
+    private static final AtomicLong qryIdGen = new AtomicLong();
+
+    /** Current queries cursors. */
+    private final ConcurrentHashMap<Long, IgniteBiTuple<QueryCursor, Iterator>> qryCurs = new ConcurrentHashMap<>();
+
+    /**
+     * @param ctx Context.
+     */
+    public QueryCommandHandler(GridKernalContext ctx) {
+        super(ctx);
+    }
+
+    /** {@inheritDoc} */
+    @Override public Collection<GridRestCommand> supportedCommands() {
+        return SUPPORTED_COMMANDS;
+    }
+
+    /** {@inheritDoc} */
+    @Override public IgniteInternalFuture<GridRestResponse> handleAsync(GridRestRequest req) {
+        assert req != null;
+
+        assert SUPPORTED_COMMANDS.contains(req.command());
+        assert req instanceof RestSqlQueryRequest : "Invalid type of query request.";
+
+        switch (req.command()) {
+            case EXECUTE_SQL_QUERY:
+            case EXECUTE_SQL_FIELDS_QUERY: {
+                return ctx.closure().callLocalSafe(
+                    new ExecuteQueryCallable(ctx, (RestSqlQueryRequest)req, qryCurs), false);
+            }
+
+            case FETCH_SQL_QUERY: {
+                return ctx.closure().callLocalSafe(
+                    new FetchQueryCallable((RestSqlQueryRequest)req, qryCurs), false);
+            }
+
+            case CLOSE_SQL_QUERY: {
+                return ctx.closure().callLocalSafe(
+                    new CloseQueryCallable((RestSqlQueryRequest)req, qryCurs), false);
+            }
+        }
+
+        return new GridFinishedFuture<>();
+    }
+
+    /**
+     * Execute query callable.
+     */
+    private static class ExecuteQueryCallable implements Callable<GridRestResponse> {
+        /** Kernal context. */
+        private GridKernalContext ctx;
+
+        /** Execute query request. */
+        private RestSqlQueryRequest req;
+
+        /** Queries cursors. */
+        private ConcurrentHashMap<Long, IgniteBiTuple<QueryCursor, Iterator>> qryCurs;
+
+        /**
+         * @param ctx Kernal context.
+         * @param req Execute query request.
+         * @param qryCurs Queries cursors.
+         */
+        public ExecuteQueryCallable(GridKernalContext ctx, RestSqlQueryRequest req,
+            ConcurrentHashMap<Long, IgniteBiTuple<QueryCursor, Iterator>> qryCurs) {
+            this.ctx = ctx;
+            this.req = req;
+            this.qryCurs = qryCurs;
+        }
+
+        /** {@inheritDoc} */
+        @Override public GridRestResponse call() throws Exception {
+            long qryId = qryIdGen.getAndIncrement();
+
+            try {
+                Query qry;
+
+                if (req.typeName() != null) {
+                    qry = new SqlQuery(req.typeName(), req.sqlQuery());
+
+                    ((SqlQuery)qry).setArgs(req.arguments());
+                }
+                else {
+                    qry = new SqlFieldsQuery(req.sqlQuery());
+
+                    ((SqlFieldsQuery)qry).setArgs(req.arguments());
+                }
+
+                IgniteCache<Object, Object> cache = ctx.grid().cache(req.cacheName());
+
+                if (cache == null)
+                    return new GridRestResponse(GridRestResponse.STATUS_FAILED,
+                        "No cache with name [cacheName=" + req.cacheName() + "]");
+
+                QueryCursor qryCur = cache.query(qry);
+
+                Iterator cur = qryCur.iterator();
+
+                qryCurs.put(qryId, new IgniteBiTuple<>(qryCur, cur));
+
+                CacheQueryResult res = createQueryResult(qryCurs, cur, req, qryId);
+
+                return new GridRestResponse(res);
+            }
+            catch (Exception e) {
+                qryCurs.remove(qryId);
+
+                return new GridRestResponse(GridRestResponse.STATUS_FAILED, e.getMessage());
+            }
+        }
+    }
+
+    /**
+     * Close query callable.
+     */
+    private static class CloseQueryCallable implements Callable<GridRestResponse> {
+        /** Execute query request. */
+        private RestSqlQueryRequest req;
+
+        /** Queries cursors. */
+        private final ConcurrentHashMap<Long, IgniteBiTuple<QueryCursor, Iterator>> qryCurs;
+
+        /**
+         * @param req Execute query request.
+         * @param qryCurs Queries cursors.
+         */
+        public CloseQueryCallable(RestSqlQueryRequest req,
+            ConcurrentHashMap<Long, IgniteBiTuple<QueryCursor, Iterator>> qryCurs) {
+            this.req = req;
+            this.qryCurs = qryCurs;
+        }
+
+        /** {@inheritDoc} */
+        @Override public GridRestResponse call() throws Exception {
+            try {
+                QueryCursor cur = qryCurs.get(req.queryId()).get1();
+
+                if (cur == null)
+                    return new GridRestResponse(GridRestResponse.STATUS_FAILED,
+                        "Cannot find query [qryId=" + req.queryId() + "]");
+
+                cur.close();
+
+                qryCurs.remove(req.queryId());
+
+                return new GridRestResponse(true);
+            }
+            catch (Exception e) {
+                qryCurs.remove(req.queryId());
+
+                return new GridRestResponse(GridRestResponse.STATUS_FAILED, e.getMessage());
+            }
+        }
+    }
+
+    /**
+     * Fetch query callable.
+     */
+    private static class FetchQueryCallable implements Callable<GridRestResponse> {
+        /** Execute query request. */
+        private RestSqlQueryRequest req;
+
+        /** Queries cursors. */
+        private final ConcurrentHashMap<Long, IgniteBiTuple<QueryCursor, Iterator>> qryCurs;
+
+        /**
+         * @param req Execute query request.
+         * @param qryCurs Queries cursors.
+         */
+        public FetchQueryCallable(RestSqlQueryRequest req,
+            ConcurrentHashMap<Long, IgniteBiTuple<QueryCursor, Iterator>> qryCurs) {
+            this.req = req;
+            this.qryCurs = qryCurs;
+        }
+
+        /** {@inheritDoc} */
+        @Override public GridRestResponse call() throws Exception {
+            try {
+                Iterator cur = qryCurs.get(req.queryId()).get2();
+
+                if (cur == null)
+                    return new GridRestResponse(GridRestResponse.STATUS_FAILED,
+                        "Cannot find query [qryId=" + req.queryId() + "]");
+
+                CacheQueryResult res = createQueryResult(qryCurs, cur, req, req.queryId());
+
+                return new GridRestResponse(res);
+            }
+            catch (Exception e) {
+                qryCurs.remove(req.queryId());
+
+                return new GridRestResponse(GridRestResponse.STATUS_FAILED, e.getMessage());
+            }
+        }
+    }
+
+    /**
+     * @param qryCurs Query cursors.
+     * @param cur Current cursor.
+     * @param req Sql request.
+     * @param qryId Query id.
+     * @return Query result with items.
+     */
+    private static CacheQueryResult createQueryResult(
+        ConcurrentHashMap<Long, IgniteBiTuple<QueryCursor, Iterator>> qryCurs,
+        Iterator cur, RestSqlQueryRequest req, Long qryId) {
+        CacheQueryResult res = new CacheQueryResult();
+
+        List<Object> items = new ArrayList<>();
+
+        for (int i = 0; i < req.pageSize() && cur.hasNext(); ++i)
+            items.add(cur.next());
+
+        res.setItems(items);
+
+        res.setLast(!cur.hasNext());
+
+        res.setQueryId(qryId);
+
+        if (!cur.hasNext())
+            qryCurs.remove(qryId);
+
+        return res;
+    }
+}

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/9bfc1842/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/handlers/version/GridVersionCommandHandler.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/handlers/version/GridVersionCommandHandler.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/handlers/version/GridVersionCommandHandler.java
index 2bfb704..bf09d30 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/handlers/version/GridVersionCommandHandler.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/handlers/version/GridVersionCommandHandler.java
@@ -30,11 +30,11 @@ import static org.apache.ignite.internal.IgniteVersionUtils.*;
 import static org.apache.ignite.internal.processors.rest.GridRestCommand.*;
 
 /**
- * Handler for {@link GridRestCommand#VERSION} command.
+ * Handler for {@link GridRestCommand#VERSION} and {@link GridRestCommand#NAME} command.
  */
 public class GridVersionCommandHandler extends GridRestCommandHandlerAdapter {
     /** Supported commands. */
-    private static final Collection<GridRestCommand> SUPPORTED_COMMANDS = U.sealList(VERSION);
+    private static final Collection<GridRestCommand> SUPPORTED_COMMANDS = U.sealList(VERSION, NAME);
 
     /**
      * @param ctx Context.
@@ -54,6 +54,14 @@ public class GridVersionCommandHandler extends GridRestCommandHandlerAdapter {
 
         assert SUPPORTED_COMMANDS.contains(req.command());
 
-        return new GridFinishedFuture<>(new GridRestResponse(VER_STR));
+        switch (req.command()){
+            case VERSION:
+                return new GridFinishedFuture<>(new GridRestResponse(VER_STR));
+
+            case NAME:
+                return new GridFinishedFuture<>(new GridRestResponse(ctx.gridName()));
+        }
+
+        return new GridFinishedFuture<>();
     }
 }

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/9bfc1842/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/request/RestSqlQueryRequest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/request/RestSqlQueryRequest.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/request/RestSqlQueryRequest.java
new file mode 100644
index 0000000..5ba3a50
--- /dev/null
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/request/RestSqlQueryRequest.java
@@ -0,0 +1,125 @@
+/*
+ * 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.rest.request;
+
+/**
+ * Sql query request.
+ */
+public class RestSqlQueryRequest extends GridRestRequest {
+    /** Sql query. */
+    private String sqlQry;
+
+    /** Sql query arguments. */
+    private Object[] args;
+
+    /** Page size. */
+    private Integer pageSize;
+
+    /** Cache name. */
+    private String cacheName;
+
+    /** Query id. */
+    private Long qryId;
+
+    /** Query type name. */
+    private String typeName;
+
+    /**
+     * @param sqlQry Sql query.
+     */
+    public void sqlQuery(String sqlQry) {
+        this.sqlQry = sqlQry;
+    }
+
+    /**
+     * @return Sql query.
+     */
+    public String sqlQuery() {
+        return sqlQry;
+    }
+
+    /**
+     * @param args Sql query arguments.
+     */
+    public void arguments(Object[] args) {
+        this.args = args;
+    }
+
+    /**
+     * @return Sql query arguments.
+     */
+    public Object[] arguments() {
+        return args;
+    }
+
+    /**
+     * @param pageSize Page size.
+     */
+    public void pageSize(Integer pageSize) {
+        this.pageSize = pageSize;
+    }
+
+    /**
+     * @return Page size.
+     */
+    public int pageSize() {
+        return pageSize;
+    }
+
+    /**
+     * @param cacheName Cache name.
+     */
+    public void cacheName(String cacheName) {
+        this.cacheName = cacheName;
+    }
+
+    /**
+     * @return Cache name.
+     */
+    public String cacheName() {
+        return cacheName;
+    }
+
+    /**
+     * @param id Query id.
+     */
+    public void queryId(Long id) {
+        this.qryId = id;
+    }
+
+    /**
+     * @return Query id.
+     */
+    public Long queryId() {
+        return qryId;
+    }
+
+    /**
+     * @param typeName Query type name.
+     */
+    public void typeName(String typeName) {
+        this.typeName = typeName;
+    }
+
+    /**
+     * @return Query type name.
+     */
+    public String typeName() {
+        return typeName;
+    }
+}

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/9bfc1842/modules/rest-http/src/main/java/org/apache/ignite/internal/processors/rest/protocols/http/jetty/GridJettyRestHandler.java
----------------------------------------------------------------------
diff --git a/modules/rest-http/src/main/java/org/apache/ignite/internal/processors/rest/protocols/http/jetty/GridJettyRestHandler.java b/modules/rest-http/src/main/java/org/apache/ignite/internal/processors/rest/protocols/http/jetty/GridJettyRestHandler.java
index fac9818..bf0f2c8 100644
--- a/modules/rest-http/src/main/java/org/apache/ignite/internal/processors/rest/protocols/http/jetty/GridJettyRestHandler.java
+++ b/modules/rest-http/src/main/java/org/apache/ignite/internal/processors/rest/protocols/http/jetty/GridJettyRestHandler.java
@@ -323,11 +323,21 @@ public class GridJettyRestHandler extends AbstractHandler {
      * @throws IgniteCheckedException If creation failed.
      */
     @Nullable private GridRestRequest createRequest(GridRestCommand cmd,
-        Map<String, Object> params,
-        ServletRequest req) throws IgniteCheckedException {
+        Map<String, Object> params, HttpServletRequest req) throws IgniteCheckedException {
         GridRestRequest restReq;
 
         switch (cmd) {
+            case GET_OR_CREATE_CACHE:
+            case DESTROY_CACHE: {
+                GridRestCacheRequest restReq0 = new GridRestCacheRequest();
+
+                restReq0.cacheName((String)params.get("cacheName"));
+
+                restReq = restReq0;
+
+                break;
+            }
+
             case ATOMIC_DECREMENT:
             case ATOMIC_INCREMENT: {
                 DataStructuresRequest restReq0 = new DataStructuresRequest();
@@ -341,15 +351,25 @@ public class GridJettyRestHandler extends AbstractHandler {
                 break;
             }
 
+            case CACHE_CONTAINS_KEY:
+            case CACHE_CONTAINS_KEYS:
             case CACHE_GET:
             case CACHE_GET_ALL:
+            case CACHE_GET_AND_PUT:
+            case CACHE_GET_AND_REPLACE:
+            case CACHE_PUT_IF_ABSENT:
+            case CACHE_GET_AND_PUT_IF_ABSENT:
             case CACHE_PUT:
             case CACHE_PUT_ALL:
             case CACHE_REMOVE:
+            case CACHE_REMOVE_VALUE:
+            case CACHE_REPLACE_VALUE:
+            case CACHE_GET_AND_REMOVE:
             case CACHE_REMOVE_ALL:
             case CACHE_ADD:
             case CACHE_CAS:
             case CACHE_METRICS:
+            case CACHE_SIZE:
             case CACHE_REPLACE:
             case CACHE_APPEND:
             case CACHE_PREPEND: {
@@ -370,7 +390,8 @@ public class GridJettyRestHandler extends AbstractHandler {
                 restReq0.cacheFlags(intValue("cacheFlags", params, 0));
                 restReq0.ttl(longValue("exp", params, null));
 
-                if (cmd == CACHE_GET_ALL || cmd == CACHE_PUT_ALL || cmd == CACHE_REMOVE_ALL) {
+                if (cmd == CACHE_GET_ALL || cmd == CACHE_PUT_ALL || cmd == CACHE_REMOVE_ALL ||
+                    cmd == CACHE_CONTAINS_KEYS) {
                     List<Object> keys = values("k", params);
                     List<Object> vals = values("v", params);
 
@@ -441,12 +462,70 @@ public class GridJettyRestHandler extends AbstractHandler {
                 break;
             }
 
+            case NAME:
             case VERSION: {
                 restReq = new GridRestRequest();
 
                 break;
             }
 
+            case EXECUTE_SQL_QUERY:
+            case EXECUTE_SQL_FIELDS_QUERY: {
+                RestSqlQueryRequest restReq0 = new RestSqlQueryRequest();
+
+                restReq0.sqlQuery((String) params.get("qry"));
+
+                restReq0.arguments(values("arg", params).toArray());
+
+                restReq0.typeName((String) params.get("type"));
+
+                String psz = (String) params.get("psz");
+
+                if (psz != null)
+                    restReq0.pageSize(Integer.parseInt(psz));
+
+                restReq0.cacheName((String)params.get("cacheName"));
+
+                restReq = restReq0;
+
+                break;
+            }
+
+            case FETCH_SQL_QUERY: {
+                RestSqlQueryRequest restReq0 = new RestSqlQueryRequest();
+
+                String qryId = (String) params.get("qryId");
+
+                if (qryId != null)
+                    restReq0.queryId(Long.parseLong(qryId));
+
+                String psz = (String) params.get("psz");
+
+                if (psz != null)
+                    restReq0.pageSize(Integer.parseInt(psz));
+
+                restReq0.cacheName((String)params.get("cacheName"));
+
+                restReq = restReq0;
+
+                break;
+            }
+
+            case CLOSE_SQL_QUERY: {
+                RestSqlQueryRequest restReq0 = new RestSqlQueryRequest();
+
+                String qryId = (String) params.get("qryId");
+
+                if (qryId != null)
+                    restReq0.queryId(Long.parseLong(qryId));
+
+                restReq0.cacheName((String)params.get("cacheName"));
+
+                restReq = restReq0;
+
+                break;
+            }
+
             default:
                 throw new IgniteCheckedException("Invalid command: " + cmd);
         }



[4/8] incubator-ignite git commit: Added putAll benchmarks.

Posted by sb...@apache.org.
Added putAll benchmarks.


Project: http://git-wip-us.apache.org/repos/asf/incubator-ignite/repo
Commit: http://git-wip-us.apache.org/repos/asf/incubator-ignite/commit/5304e6ee
Tree: http://git-wip-us.apache.org/repos/asf/incubator-ignite/tree/5304e6ee
Diff: http://git-wip-us.apache.org/repos/asf/incubator-ignite/diff/5304e6ee

Branch: refs/heads/ignite-gg-10295
Commit: 5304e6ee47970c6989784a2ab47f875437883cfa
Parents: f22de3e
Author: nikolay_tikhonov <nt...@gridgain.com>
Authored: Mon Jul 27 13:46:47 2015 +0300
Committer: nikolay_tikhonov <nt...@gridgain.com>
Committed: Mon Jul 27 13:46:47 2015 +0300

----------------------------------------------------------------------
 modules/yardstick/config/benchmark.properties   |  6 +-
 .../yardstick/IgniteBenchmarkArguments.java     | 22 +++++++
 .../yardstick/cache/IgnitePutAllBenchmark.java  | 67 +++++++++++++++++++
 .../cache/IgnitePutAllTxBenchmark.java          | 68 ++++++++++++++++++++
 4 files changed, 162 insertions(+), 1 deletion(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/5304e6ee/modules/yardstick/config/benchmark.properties
----------------------------------------------------------------------
diff --git a/modules/yardstick/config/benchmark.properties b/modules/yardstick/config/benchmark.properties
index edda9ee..d691f1e 100644
--- a/modules/yardstick/config/benchmark.properties
+++ b/modules/yardstick/config/benchmark.properties
@@ -86,5 +86,9 @@ CONFIGS="\
 -cfg ${SCRIPT_DIR}/../config/ignite-localhost-config.xml -nn ${nodesNum} -b 1 -w 60 -d 300 -t 64 -sm PRIMARY_SYNC -dn IgnitePutTxOffHeapValuesBenchmark -sn IgniteNode -ds ${ver}tx-put-offheap-val-1-backup,\
 -cfg ${SCRIPT_DIR}/../config/ignite-localhost-config.xml -nn ${nodesNum} -b 1 -w 60 -d 300 -t 64 -sm PRIMARY_SYNC -dn IgniteSqlQueryOffHeapBenchmark -sn IgniteNode -ds ${ver}sql-query-offheap-1-backup,\
 -cfg ${SCRIPT_DIR}/../config/ignite-localhost-config.xml -nn ${nodesNum} -b 1 -w 60 -d 300 -t 64 -sm PRIMARY_SYNC -dn IgniteSqlQueryJoinOffHeapBenchmark -sn IgniteNode -ds ${ver}sql-query-join-offheap-1-backup,\
--cfg ${SCRIPT_DIR}/../config/ignite-localhost-config.xml -nn ${nodesNum} -b 1 -w 60 -d 300 -t 64 -sm PRIMARY_SYNC -dn IgniteSqlQueryPutOffHeapBenchmark -sn IgniteNode -ds ${ver}sql-query-put-offheap-1-backup\
+-cfg ${SCRIPT_DIR}/../config/ignite-localhost-config.xml -nn ${nodesNum} -b 1 -w 60 -d 300 -t 64 -sm PRIMARY_SYNC -dn IgniteSqlQueryPutOffHeapBenchmark -sn IgniteNode -ds ${ver}sql-query-put-offheap-1-backup,\
+-cfg ${SCRIPT_DIR}/../config/ignite-localhost-config.xml -nn ${nodesNum} -b 1 -w 60 -d 300 -t 64 -bs 1000 -sm PRIMARY_SYNC -dn IgnitePutAllBenchmark -sn IgniteNode -ds ${ver}atomic-putAll-1-backup,\
+-cfg ${SCRIPT_DIR}/../config/ignite-localhost-config.xml -nn ${nodesNum} -b 1 -w 60 -d 300 -t 64 -bs 1000 -col -sm PRIMARY_SYNC -dn IgnitePutAllBenchmark -sn IgniteNode -ds ${ver}atomic-collocated-putAll-1-backup,\
+-cfg ${SCRIPT_DIR}/../config/ignite-localhost-config.xml -nn ${nodesNum} -b 1 -w 60 -d 300 -t 64 -bs 1000 -sm PRIMARY_SYNC -dn IgnitePutAllTxBenchmark -sn IgniteNode -ds ${ver}atomic-putAll-tx-1-backup,\
+-cfg ${SCRIPT_DIR}/../config/ignite-localhost-config.xml -nn ${nodesNum} -b 1 -w 60 -d 300 -t 64 -bs 1000 -col -sm PRIMARY_SYNC -dn IgnitePutAllTxBenchmark -sn IgniteNode -ds ${ver}atomic-collocated-putAll-tx-1-backup\
 "

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/5304e6ee/modules/yardstick/src/main/java/org/apache/ignite/yardstick/IgniteBenchmarkArguments.java
----------------------------------------------------------------------
diff --git a/modules/yardstick/src/main/java/org/apache/ignite/yardstick/IgniteBenchmarkArguments.java b/modules/yardstick/src/main/java/org/apache/ignite/yardstick/IgniteBenchmarkArguments.java
index b21bed3..1562b26 100644
--- a/modules/yardstick/src/main/java/org/apache/ignite/yardstick/IgniteBenchmarkArguments.java
+++ b/modules/yardstick/src/main/java/org/apache/ignite/yardstick/IgniteBenchmarkArguments.java
@@ -98,6 +98,14 @@ public class IgniteBenchmarkArguments {
     @Parameter(names = {"-wb", "--writeBehind"}, description = "Enable or disable writeBehind for cache store")
     private boolean writeBehind;
 
+    /** */
+    @Parameter(names = {"-bs", "--batchSize"}, description = "Batch size")
+    private int batch = 500;
+
+    /** */
+    @Parameter(names = {"-col", "--collocated"}, description = "Collocated")
+    private boolean collocated;
+
     /**
      * @return Transaction concurrency.
      */
@@ -232,6 +240,20 @@ public class IgniteBenchmarkArguments {
     }
 
     /**
+     * @return Batch size.
+     */
+    public int batch() {
+        return batch;
+    }
+
+    /**
+     * @return Collocated.
+     */
+    public boolean collocated() {
+        return collocated;
+    }
+
+    /**
      * @return Description.
      */
     public String description() {

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/5304e6ee/modules/yardstick/src/main/java/org/apache/ignite/yardstick/cache/IgnitePutAllBenchmark.java
----------------------------------------------------------------------
diff --git a/modules/yardstick/src/main/java/org/apache/ignite/yardstick/cache/IgnitePutAllBenchmark.java b/modules/yardstick/src/main/java/org/apache/ignite/yardstick/cache/IgnitePutAllBenchmark.java
new file mode 100644
index 0000000..6b30875
--- /dev/null
+++ b/modules/yardstick/src/main/java/org/apache/ignite/yardstick/cache/IgnitePutAllBenchmark.java
@@ -0,0 +1,67 @@
+/*
+ * 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.yardstick.cache;
+
+import org.apache.ignite.*;
+import org.apache.ignite.cache.affinity.*;
+import org.apache.ignite.cluster.*;
+import org.yardstickframework.*;
+
+import java.util.*;
+
+/**
+ * Ignite benchmark that performs putAll operations.
+ */
+public class IgnitePutAllBenchmark extends IgniteCacheAbstractBenchmark {
+    /** Affinity mapper. */
+    private Affinity<Integer> aff;
+
+    /** {@inheritDoc} */
+    @Override public void setUp(BenchmarkConfiguration cfg) throws Exception {
+        super.setUp(cfg);
+
+        aff = ignite().affinity("atomic");
+    }
+
+    /** {@inheritDoc} */
+    @Override public boolean test(Map<Object, Object> ctx) throws Exception {
+        SortedMap<Integer, Integer> vals = new TreeMap<>();
+
+        ClusterNode node = args.collocated() ? aff.mapKeyToNode(nextRandom(args.range())) : null;
+
+        for (int i = 0; i < args.batch(); ) {
+            int key = nextRandom(args.range());
+
+            if (args.collocated() && !aff.isPrimary(node, key))
+                continue;
+
+            ++i;
+
+            vals.put(key, key);
+        }
+
+        cache.putAll(vals);
+
+        return true;
+    }
+
+    /** {@inheritDoc} */
+    @Override protected IgniteCache<Integer, Object> cache() {
+        return ignite().cache("atomic");
+    }
+}

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/5304e6ee/modules/yardstick/src/main/java/org/apache/ignite/yardstick/cache/IgnitePutAllTxBenchmark.java
----------------------------------------------------------------------
diff --git a/modules/yardstick/src/main/java/org/apache/ignite/yardstick/cache/IgnitePutAllTxBenchmark.java b/modules/yardstick/src/main/java/org/apache/ignite/yardstick/cache/IgnitePutAllTxBenchmark.java
new file mode 100644
index 0000000..01890c9
--- /dev/null
+++ b/modules/yardstick/src/main/java/org/apache/ignite/yardstick/cache/IgnitePutAllTxBenchmark.java
@@ -0,0 +1,68 @@
+/*
+ * 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.yardstick.cache;
+
+import org.apache.ignite.*;
+import org.apache.ignite.cache.affinity.*;
+import org.apache.ignite.cluster.*;
+import org.yardstickframework.*;
+
+import java.util.*;
+
+/**
+ * Ignite benchmark that performs transactional putAll operations.
+ */
+public class IgnitePutAllTxBenchmark extends IgniteCacheAbstractBenchmark {
+    /** Affinity mapper. */
+    private Affinity<Integer> aff;
+
+    /** {@inheritDoc} */
+    @Override public void setUp(BenchmarkConfiguration cfg) throws Exception {
+        super.setUp(cfg);
+
+        aff = ignite().affinity("tx");
+    }
+
+    /** {@inheritDoc} */
+    @Override public boolean test(Map<Object, Object> ctx) throws Exception {
+        SortedMap<Integer, Integer> vals = new TreeMap<>();
+
+        ClusterNode node = args.collocated() ? aff.mapKeyToNode(nextRandom(args.range())) : null;
+
+        for (int i = 0; i < args.batch(); ) {
+            int key = nextRandom(args.range());
+
+            if (args.collocated() && !aff.isPrimary(node, key))
+                continue;
+
+            ++i;
+
+            vals.put(key, key);
+        }
+
+        // Implicit transaction is used.
+        cache.putAll(vals);
+
+        return true;
+    }
+
+    /** {@inheritDoc} */
+    @Override protected IgniteCache<Integer, Object> cache() {
+        return ignite().cache("tx");
+    }
+}


[5/8] incubator-ignite git commit: #ignite-961: Add sql fields metadata to rest query response.

Posted by sb...@apache.org.
#ignite-961:  Add sql fields metadata to rest query response.


Project: http://git-wip-us.apache.org/repos/asf/incubator-ignite/repo
Commit: http://git-wip-us.apache.org/repos/asf/incubator-ignite/commit/201e208e
Tree: http://git-wip-us.apache.org/repos/asf/incubator-ignite/tree/201e208e
Diff: http://git-wip-us.apache.org/repos/asf/incubator-ignite/diff/201e208e

Branch: refs/heads/ignite-gg-10295
Commit: 201e208e82b0d0c8f8fdb16747d703eedc7f95db
Parents: f22de3e
Author: ivasilinets <iv...@gridgain.com>
Authored: Mon Jul 27 13:58:37 2015 +0300
Committer: ivasilinets <iv...@gridgain.com>
Committed: Mon Jul 27 13:58:37 2015 +0300

----------------------------------------------------------------------
 .../JettyRestProcessorAbstractSelfTest.java     |  41 +++++-
 .../query/CacheQueryFieldsMetaResult.java       | 133 +++++++++++++++++++
 .../rest/handlers/query/CacheQueryResult.java   |  19 +++
 .../handlers/query/QueryCommandHandler.java     |  21 +++
 4 files changed, 212 insertions(+), 2 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/201e208e/modules/clients/src/test/java/org/apache/ignite/internal/processors/rest/JettyRestProcessorAbstractSelfTest.java
----------------------------------------------------------------------
diff --git a/modules/clients/src/test/java/org/apache/ignite/internal/processors/rest/JettyRestProcessorAbstractSelfTest.java b/modules/clients/src/test/java/org/apache/ignite/internal/processors/rest/JettyRestProcessorAbstractSelfTest.java
index 160046c..8ce070f 100644
--- a/modules/clients/src/test/java/org/apache/ignite/internal/processors/rest/JettyRestProcessorAbstractSelfTest.java
+++ b/modules/clients/src/test/java/org/apache/ignite/internal/processors/rest/JettyRestProcessorAbstractSelfTest.java
@@ -650,7 +650,7 @@ public abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestPro
 
         jsonEquals(ret, cacheBulkPattern(true, true));
 
-        ret = content(F.asMap("cmd", "rmvall"));
+        ret = content(F.asMap("cmd", GridRestCommand.CACHE_REMOVE_ALL.key()));
 
         assertNotNull(ret);
         assertTrue(!ret.isEmpty());
@@ -1001,7 +1001,7 @@ public abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestPro
      * @throws Exception If failed.
      */
     public void testVersion() throws Exception {
-        String ret = content(F.asMap("cmd", "version"));
+        String ret = content(F.asMap("cmd", GridRestCommand.VERSION.key()));
 
         assertNotNull(ret);
         assertTrue(!ret.isEmpty());
@@ -1119,6 +1119,43 @@ public abstract class JettyRestProcessorAbstractSelfTest extends AbstractRestPro
     /**
      * @throws Exception If failed.
      */
+    public void testSqlFieldsMetadataQuery() throws Exception {
+        String qry = "select firstName, lastName from Person";
+
+        Map<String, String> params = new HashMap<>();
+        params.put("cmd", GridRestCommand.EXECUTE_SQL_FIELDS_QUERY.key());
+        params.put("psz", "10");
+        params.put("cacheName", "person");
+        params.put("qry", URLEncoder.encode(qry));
+
+        String ret = content(params);
+
+        assertNotNull(ret);
+        assertTrue(!ret.isEmpty());
+
+        JSONObject json = JSONObject.fromObject(ret);
+
+        List items = (List)((Map)json.get("response")).get("items");
+
+        List meta = (List)((Map)json.get("response")).get("fieldsMetadata");
+
+        assertEquals(4, items.size());
+
+        assertEquals(2, meta.size());
+
+        JSONObject o = (JSONObject)meta.get(0);
+
+        assertEquals("FIRSTNAME", o.get("fieldName"));
+        assertEquals("java.lang.String", o.get("fieldTypeName"));
+        assertEquals("person", o.get("schemaName"));
+        assertEquals("PERSON", o.get("typeName"));
+
+        assertFalse(queryCursorFound());
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
     public void testQueryClose() throws Exception {
         String qry = "salary > ? and salary <= ?";
 

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/201e208e/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/handlers/query/CacheQueryFieldsMetaResult.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/handlers/query/CacheQueryFieldsMetaResult.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/handlers/query/CacheQueryFieldsMetaResult.java
new file mode 100644
index 0000000..8c4ddb0
--- /dev/null
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/handlers/query/CacheQueryFieldsMetaResult.java
@@ -0,0 +1,133 @@
+/*
+ * 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.rest.handlers.query;
+
+import org.apache.ignite.internal.processors.query.*;
+import org.apache.ignite.internal.util.typedef.internal.*;
+
+import java.io.*;
+
+/**
+ * Cache query fields metadata.
+ */
+public class CacheQueryFieldsMetaResult implements Externalizable {
+    /** */
+    private static final long serialVersionUID = 0L;
+
+    /** Schema name. */
+    private String schemaName;
+
+    /** Type name. */
+    private String typeName;
+
+    /** Name. */
+    private String fieldName;
+
+    /** Type. */
+    private String fieldTypeName;
+
+    /**
+     * Empty constructor for Externalizable.
+     */
+    public CacheQueryFieldsMetaResult() {
+        // No-op.
+    }
+
+    /**
+     * @param meta Metadata
+     */
+    public CacheQueryFieldsMetaResult(GridQueryFieldMetadata meta) {
+        schemaName = meta.schemaName();
+        typeName = meta.typeName();
+        fieldName = meta.fieldName();
+        fieldTypeName = meta.fieldTypeName();
+    }
+
+    /**
+     * @return Schema name.
+     */
+    public String getSchemaName() {
+        return schemaName;
+    }
+
+    /**
+     * @param schemaName Schema name.
+     */
+    public void setSchemaName(String schemaName) {
+        this.schemaName = schemaName;
+    }
+
+    /**
+     * @return Type name.
+     */
+    public String getTypeName() {
+        return typeName;
+    }
+
+    /**
+     * @param typeName Type name.
+     */
+    public void setTypeName(String typeName) {
+        this.typeName = typeName;
+    }
+
+    /**
+     * @return Field name.
+     */
+    public String getFieldName() {
+        return fieldName;
+    }
+
+    /**
+     * @param fieldName Field name.
+     */
+    public void setFieldName(String fieldName) {
+        this.fieldName = fieldName;
+    }
+
+
+    /**
+     * @return Field type name.
+     */
+    public String getFieldTypeName() {
+        return fieldTypeName;
+    }
+
+    /**
+     * @param fieldName Field name.
+     */
+    public void setFieldTypeName(String fieldName) {
+        this.fieldName = fieldName;
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeExternal(ObjectOutput out) throws IOException {
+        U.writeString(out, schemaName);
+        U.writeString(out, typeName);
+        U.writeString(out, fieldName);
+        U.writeString(out, fieldTypeName);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void readExternal(ObjectInput in) throws IOException, ClassNotFoundException {
+        schemaName = U.readString(in);
+        typeName = U.readString(in);
+        fieldName = U.readString(in);
+        fieldTypeName = U.readString(in);
+    }
+}

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/201e208e/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/handlers/query/CacheQueryResult.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/handlers/query/CacheQueryResult.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/handlers/query/CacheQueryResult.java
index 3e49576..544a094 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/handlers/query/CacheQueryResult.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/handlers/query/CacheQueryResult.java
@@ -35,6 +35,9 @@ public class CacheQueryResult implements Externalizable {
     /** Result items. */
     private Collection<?> items;
 
+    /** Fields metadata. */
+    private Collection<?> fieldsMeta;
+
     /** Last flag. */
     private boolean last;
 
@@ -67,6 +70,20 @@ public class CacheQueryResult implements Externalizable {
     }
 
     /**
+     * @param fieldsMeta Fields metadata.
+     */
+    public void setFieldsMetadata(Collection<?> fieldsMeta) {
+        this.fieldsMeta = fieldsMeta;
+    }
+
+    /**
+     * @return Fields metadata.
+     */
+    public Collection<?> getFieldsMetadata() {
+        return fieldsMeta;
+    }
+
+    /**
      * @return Last flag.
      */
     public boolean getLast() {
@@ -90,6 +107,7 @@ public class CacheQueryResult implements Externalizable {
         out.writeBoolean(last);
         out.writeLong(qryId);
         U.writeCollection(out, items);
+        U.writeCollection(out, fieldsMeta);
     }
 
     /** {@inheritDoc} */
@@ -97,5 +115,6 @@ public class CacheQueryResult implements Externalizable {
         last = in.readBoolean();
         qryId = in.readLong();
         items = U.readCollection(in);
+        fieldsMeta = U.readCollection(in);
     }
 }

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/201e208e/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/handlers/query/QueryCommandHandler.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/handlers/query/QueryCommandHandler.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/handlers/query/QueryCommandHandler.java
index defca37..59f95c9 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/handlers/query/QueryCommandHandler.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/rest/handlers/query/QueryCommandHandler.java
@@ -20,6 +20,8 @@ package org.apache.ignite.internal.processors.rest.handlers.query;
 import org.apache.ignite.*;
 import org.apache.ignite.cache.query.*;
 import org.apache.ignite.internal.*;
+import org.apache.ignite.internal.processors.cache.*;
+import org.apache.ignite.internal.processors.query.*;
 import org.apache.ignite.internal.processors.rest.*;
 import org.apache.ignite.internal.processors.rest.handlers.*;
 import org.apache.ignite.internal.processors.rest.request.*;
@@ -146,6 +148,10 @@ public class QueryCommandHandler extends GridRestCommandHandlerAdapter {
 
                 CacheQueryResult res = createQueryResult(qryCurs, cur, req, qryId);
 
+                List<GridQueryFieldMetadata> fieldsMeta = ((QueryCursorImpl<?>) qryCur).fieldsMeta();
+
+                res.setFieldsMetadata(convertMetadata(fieldsMeta));
+
                 return new GridRestResponse(res);
             }
             catch (Exception e) {
@@ -154,6 +160,21 @@ public class QueryCommandHandler extends GridRestCommandHandlerAdapter {
                 return new GridRestResponse(GridRestResponse.STATUS_FAILED, e.getMessage());
             }
         }
+
+        /**
+         * @param meta Internal query field metadata.
+         * @return Rest query field metadata.
+         */
+        private Collection<CacheQueryFieldsMetaResult> convertMetadata(Collection<GridQueryFieldMetadata> meta) {
+            List<CacheQueryFieldsMetaResult> res = new ArrayList<>();
+
+            if (meta != null) {
+                for (GridQueryFieldMetadata info : meta)
+                    res.add(new CacheQueryFieldsMetaResult(info));
+            }
+
+            return res;
+        }
     }
 
     /**


[7/8] incubator-ignite git commit: #ignite-gg-10610: Security hole if DataStreamer is used for populating the cache

Posted by sb...@apache.org.
#ignite-gg-10610: Security hole if DataStreamer is used for populating the cache


Project: http://git-wip-us.apache.org/repos/asf/incubator-ignite/repo
Commit: http://git-wip-us.apache.org/repos/asf/incubator-ignite/commit/5288b2d8
Tree: http://git-wip-us.apache.org/repos/asf/incubator-ignite/tree/5288b2d8
Diff: http://git-wip-us.apache.org/repos/asf/incubator-ignite/diff/5288b2d8

Branch: refs/heads/ignite-gg-10295
Commit: 5288b2d8b882bbb86d69e1019821d51803685861
Parents: a127756
Author: ivasilinets <iv...@gridgain.com>
Authored: Wed Jul 29 15:27:31 2015 +0300
Committer: ivasilinets <iv...@gridgain.com>
Committed: Wed Jul 29 15:27:31 2015 +0300

----------------------------------------------------------------------
 .../datastreamer/DataStreamerImpl.java          | 22 ++++++++++++++++++++
 .../datastreamer/DataStreamerUpdateJob.java     | 20 +++++++++++++++++-
 2 files changed, 41 insertions(+), 1 deletion(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/5288b2d8/modules/core/src/main/java/org/apache/ignite/internal/processors/datastreamer/DataStreamerImpl.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/datastreamer/DataStreamerImpl.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/datastreamer/DataStreamerImpl.java
index 605f478..5fae676 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/datastreamer/DataStreamerImpl.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/datastreamer/DataStreamerImpl.java
@@ -39,6 +39,7 @@ import org.apache.ignite.internal.util.tostring.*;
 import org.apache.ignite.internal.util.typedef.*;
 import org.apache.ignite.internal.util.typedef.internal.*;
 import org.apache.ignite.lang.*;
+import org.apache.ignite.plugin.security.*;
 import org.apache.ignite.stream.*;
 import org.jetbrains.annotations.*;
 import org.jsr166.*;
@@ -413,6 +414,8 @@ public class DataStreamerImpl<K, V> implements IgniteDataStreamer<K, V>, Delayed
     @Override public IgniteFuture<?> addData(Collection<? extends Map.Entry<K, V>> entries) {
         A.notEmpty(entries, "entries");
 
+        checkSecurityPermission(SecurityPermission.CACHE_PUT);
+
         enterBusy();
 
         try {
@@ -520,6 +523,11 @@ public class DataStreamerImpl<K, V> implements IgniteDataStreamer<K, V>, Delayed
     @Override public IgniteFuture<?> addData(K key, V val) {
         A.notNull(key, "key");
 
+        if (val == null)
+            checkSecurityPermission(SecurityPermission.CACHE_REMOVE);
+        else
+            checkSecurityPermission(SecurityPermission.CACHE_PUT);
+
         KeyCacheObject key0 = cacheObjProc.toCacheKeyObject(cacheObjCtx, key, true);
         CacheObject val0 = cacheObjProc.toCacheObject(cacheObjCtx, val, true);
 
@@ -980,6 +988,20 @@ public class DataStreamerImpl<K, V> implements IgniteDataStreamer<K, V>, Delayed
     }
 
     /**
+     * Check permissions for streaming.
+     *
+     * @param perm Security permission.
+     * @throws org.apache.ignite.plugin.security.SecurityException If permissions are not enough for streaming.
+     */
+    private void checkSecurityPermission(SecurityPermission perm)
+        throws org.apache.ignite.plugin.security.SecurityException{
+        if (!ctx.security().enabled())
+            return;
+
+        ctx.security().authorize(cacheName, perm, null);
+    }
+
+    /**
      *
      */
     private class Buffer {

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/5288b2d8/modules/core/src/main/java/org/apache/ignite/internal/processors/datastreamer/DataStreamerUpdateJob.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/datastreamer/DataStreamerUpdateJob.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/datastreamer/DataStreamerUpdateJob.java
index 21ba3ac..9e0703a 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/datastreamer/DataStreamerUpdateJob.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/datastreamer/DataStreamerUpdateJob.java
@@ -22,6 +22,7 @@ import org.apache.ignite.internal.*;
 import org.apache.ignite.internal.processors.cache.*;
 import org.apache.ignite.internal.util.lang.*;
 import org.apache.ignite.internal.util.typedef.*;
+import org.apache.ignite.plugin.security.*;
 import org.apache.ignite.stream.*;
 import org.jetbrains.annotations.*;
 
@@ -106,8 +107,13 @@ class DataStreamerUpdateJob implements GridPlainCallable<Object> {
 
                 CacheObject val = e.getValue();
 
-                if (val != null)
+                if (val != null) {
+                    checkSecurityPermission(SecurityPermission.CACHE_PUT);
+
                     val.finishUnmarshal(cctx.cacheObjectContext(), cctx.deploy().globalLoader());
+                }
+                else
+                    checkSecurityPermission(SecurityPermission.CACHE_REMOVE);
             }
 
             if (unwrapEntries()) {
@@ -139,4 +145,16 @@ class DataStreamerUpdateJob implements GridPlainCallable<Object> {
     private boolean unwrapEntries() {
         return !(rcvr instanceof DataStreamerCacheUpdaters.InternalUpdater);
     }
+
+    /**
+     * @param perm Security permission.
+     * @throws org.apache.ignite.plugin.security.SecurityException If permission is not enough.
+     */
+    private void checkSecurityPermission(SecurityPermission perm)
+        throws org.apache.ignite.plugin.security.SecurityException {
+        if (!ctx.security().enabled())
+            return;
+
+        ctx.security().authorize(cacheName, perm, null);
+    }
 }


[6/8] incubator-ignite git commit: Merge branch 'master' of https://git-wip-us.apache.org/repos/asf/incubator-ignite

Posted by sb...@apache.org.
Merge branch 'master' of https://git-wip-us.apache.org/repos/asf/incubator-ignite


Project: http://git-wip-us.apache.org/repos/asf/incubator-ignite/repo
Commit: http://git-wip-us.apache.org/repos/asf/incubator-ignite/commit/a1277562
Tree: http://git-wip-us.apache.org/repos/asf/incubator-ignite/tree/a1277562
Diff: http://git-wip-us.apache.org/repos/asf/incubator-ignite/diff/a1277562

Branch: refs/heads/ignite-gg-10295
Commit: a12775621f85ea6e203761b116a06bf79ea12b13
Parents: 201e208 5304e6e
Author: ivasilinets <iv...@gridgain.com>
Authored: Mon Jul 27 13:59:14 2015 +0300
Committer: ivasilinets <iv...@gridgain.com>
Committed: Mon Jul 27 13:59:14 2015 +0300

----------------------------------------------------------------------
 modules/yardstick/config/benchmark.properties   |  6 +-
 .../yardstick/IgniteBenchmarkArguments.java     | 22 +++++++
 .../yardstick/cache/IgnitePutAllBenchmark.java  | 67 +++++++++++++++++++
 .../cache/IgnitePutAllTxBenchmark.java          | 68 ++++++++++++++++++++
 4 files changed, 162 insertions(+), 1 deletion(-)
----------------------------------------------------------------------



[3/8] incubator-ignite git commit: removing Google Compute Engine buckets on test finish

Posted by sb...@apache.org.
removing Google Compute Engine buckets on test finish


Project: http://git-wip-us.apache.org/repos/asf/incubator-ignite/repo
Commit: http://git-wip-us.apache.org/repos/asf/incubator-ignite/commit/f22de3ea
Tree: http://git-wip-us.apache.org/repos/asf/incubator-ignite/tree/f22de3ea
Diff: http://git-wip-us.apache.org/repos/asf/incubator-ignite/diff/f22de3ea

Branch: refs/heads/ignite-gg-10295
Commit: f22de3ea35b1506c8bcf4d2f7bb6d423a0b3225d
Parents: 9bfc184
Author: Denis Magda <dm...@gridgain.com>
Authored: Mon Jul 27 11:06:30 2015 +0300
Committer: Denis Magda <dm...@gridgain.com>
Committed: Mon Jul 27 11:06:30 2015 +0300

----------------------------------------------------------------------
 .../TcpDiscoveryIpFinderAbstractSelfTest.java   |  2 +-
 .../gce/TcpDiscoveryGoogleStorageIpFinder.java  | 18 ++++++++++++
 ...pDiscoveryGoogleStorageIpFinderSelfTest.java | 29 +++++++++++++++++++-
 3 files changed, 47 insertions(+), 2 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/f22de3ea/modules/core/src/test/java/org/apache/ignite/spi/discovery/tcp/ipfinder/TcpDiscoveryIpFinderAbstractSelfTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/spi/discovery/tcp/ipfinder/TcpDiscoveryIpFinderAbstractSelfTest.java b/modules/core/src/test/java/org/apache/ignite/spi/discovery/tcp/ipfinder/TcpDiscoveryIpFinderAbstractSelfTest.java
index b5b8c63..ba660b8 100644
--- a/modules/core/src/test/java/org/apache/ignite/spi/discovery/tcp/ipfinder/TcpDiscoveryIpFinderAbstractSelfTest.java
+++ b/modules/core/src/test/java/org/apache/ignite/spi/discovery/tcp/ipfinder/TcpDiscoveryIpFinderAbstractSelfTest.java
@@ -31,7 +31,7 @@ import java.util.*;
 public abstract class TcpDiscoveryIpFinderAbstractSelfTest<T extends TcpDiscoveryIpFinder>
     extends GridCommonAbstractTest {
     /** */
-    private T finder;
+    protected T finder;
 
     /**
      * Constructor.

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/f22de3ea/modules/gce/src/main/java/org/apache/ignite/spi/discovery/tcp/ipfinder/gce/TcpDiscoveryGoogleStorageIpFinder.java
----------------------------------------------------------------------
diff --git a/modules/gce/src/main/java/org/apache/ignite/spi/discovery/tcp/ipfinder/gce/TcpDiscoveryGoogleStorageIpFinder.java b/modules/gce/src/main/java/org/apache/ignite/spi/discovery/tcp/ipfinder/gce/TcpDiscoveryGoogleStorageIpFinder.java
index 48991e8..c2b1a91 100644
--- a/modules/gce/src/main/java/org/apache/ignite/spi/discovery/tcp/ipfinder/gce/TcpDiscoveryGoogleStorageIpFinder.java
+++ b/modules/gce/src/main/java/org/apache/ignite/spi/discovery/tcp/ipfinder/gce/TcpDiscoveryGoogleStorageIpFinder.java
@@ -380,4 +380,22 @@ public class TcpDiscoveryGoogleStorageIpFinder extends TcpDiscoveryIpFinderAdapt
 
         return new InetSocketAddress(res[0], port);
     }
+
+    /**
+     * Used by TEST SUITES only. Called through reflection.
+     *
+     * @param bucketName Bucket to delete.
+     */
+    private void removeBucket(String bucketName) {
+        init();
+
+        try {
+            Storage.Buckets.Delete deleteBucket = storage.buckets().delete(bucketName);
+
+            deleteBucket.execute();
+        }
+        catch (Exception e) {
+            throw new IgniteSpiException("Failed to remove the bucket: " + bucketName, e);
+        }
+    }
 }

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/f22de3ea/modules/gce/src/test/java/org/apache/ignite/spi/discovery/tcp/ipfinder/gce/TcpDiscoveryGoogleStorageIpFinderSelfTest.java
----------------------------------------------------------------------
diff --git a/modules/gce/src/test/java/org/apache/ignite/spi/discovery/tcp/ipfinder/gce/TcpDiscoveryGoogleStorageIpFinderSelfTest.java b/modules/gce/src/test/java/org/apache/ignite/spi/discovery/tcp/ipfinder/gce/TcpDiscoveryGoogleStorageIpFinderSelfTest.java
index 2fa0d43..f8929cc 100644
--- a/modules/gce/src/test/java/org/apache/ignite/spi/discovery/tcp/ipfinder/gce/TcpDiscoveryGoogleStorageIpFinderSelfTest.java
+++ b/modules/gce/src/test/java/org/apache/ignite/spi/discovery/tcp/ipfinder/gce/TcpDiscoveryGoogleStorageIpFinderSelfTest.java
@@ -21,6 +21,7 @@ import org.apache.ignite.internal.util.typedef.internal.U;
 import org.apache.ignite.spi.discovery.tcp.ipfinder.*;
 import org.apache.ignite.testsuites.*;
 
+import java.lang.reflect.*;
 import java.net.InetAddress;
 import java.net.InetSocketAddress;
 import java.util.Collection;
@@ -30,6 +31,9 @@ import java.util.Collection;
  */
 public class TcpDiscoveryGoogleStorageIpFinderSelfTest
     extends TcpDiscoveryIpFinderAbstractSelfTest<TcpDiscoveryGoogleStorageIpFinder> {
+    /** Bucket name. */
+    private static String bucketName;
+
     /**
      * Constructor.
      *
@@ -40,6 +44,29 @@ public class TcpDiscoveryGoogleStorageIpFinderSelfTest
     }
 
     /** {@inheritDoc} */
+    @Override protected void beforeTestsStarted() throws Exception {
+        bucketName = "ip-finder-test-bucket-" + InetAddress.getLocalHost().getAddress()[3];
+
+        super.beforeTestsStarted();
+    }
+
+    /** {@inheritDoc} */
+    @Override protected void afterTestsStopped() throws Exception {
+        try {
+            Method method = TcpDiscoveryGoogleStorageIpFinder.class.getDeclaredMethod("removeBucket", String.class);
+
+            method.setAccessible(true);
+
+            method.invoke(finder, bucketName);
+        }
+        catch (Exception e) {
+            log.warning("Failed to remove bucket on GCE [bucketName=" + bucketName + ", mes=" + e.getMessage() + ']');
+        }
+
+        super.afterTestsStopped();
+    }
+
+    /** {@inheritDoc} */
     @Override protected TcpDiscoveryGoogleStorageIpFinder ipFinder() throws Exception {
         TcpDiscoveryGoogleStorageIpFinder finder = new TcpDiscoveryGoogleStorageIpFinder();
 
@@ -52,7 +79,7 @@ public class TcpDiscoveryGoogleStorageIpFinderSelfTest
         finder.setProjectName(IgniteGCETestSuite.getProjectName());
 
         // Bucket name must be unique across the whole GCE platform.
-        finder.setBucketName("ip-finder-test-bucket-" + InetAddress.getLocalHost().getAddress()[3]);
+        finder.setBucketName(bucketName);
 
         for (int i = 0; i < 5; i++) {
             Collection<InetSocketAddress> addrs = finder.getRegisteredAddresses();