You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@ignite.apache.org by se...@apache.org on 2015/06/17 19:34:14 UTC

[01/10] incubator-ignite git commit: ignite-1020 create GridDhtTxLocal with storeEnabled=true

Repository: incubator-ignite
Updated Branches:
  refs/heads/ignite-484-1 42982382e -> ed43dfe1c


ignite-1020 create GridDhtTxLocal with storeEnabled=true


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

Branch: refs/heads/ignite-484-1
Commit: 7fd14411c36b6de7b79841c9af81506780279a10
Parents: e3750b7
Author: sboikov <sb...@gridgain.com>
Authored: Tue Jun 16 17:14:54 2015 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Tue Jun 16 17:14:54 2015 +0300

----------------------------------------------------------------------
 .../cache/distributed/dht/GridDhtTransactionalCacheAdapter.java    | 2 +-
 .../internal/processors/cache/transactions/IgniteTxHandler.java    | 2 +-
 2 files changed, 2 insertions(+), 2 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/7fd14411/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/GridDhtTransactionalCacheAdapter.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/GridDhtTransactionalCacheAdapter.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/GridDhtTransactionalCacheAdapter.java
index 703daf9..4f081bf 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/GridDhtTransactionalCacheAdapter.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/GridDhtTransactionalCacheAdapter.java
@@ -859,7 +859,7 @@ public abstract class GridDhtTransactionalCacheAdapter<K, V> extends GridDhtCach
                                         req.isolation(),
                                         req.timeout(),
                                         req.isInvalidate(),
-                                        false,
+                                        true,
                                         req.txSize(),
                                         null,
                                         req.subjectId(),

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/7fd14411/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/transactions/IgniteTxHandler.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/transactions/IgniteTxHandler.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/transactions/IgniteTxHandler.java
index 01662ef..e481e25 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/transactions/IgniteTxHandler.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/transactions/IgniteTxHandler.java
@@ -325,7 +325,7 @@ public class IgniteTxHandler {
                     req.isolation(),
                     req.timeout(),
                     req.isInvalidate(),
-                    false,
+                    true,
                     req.txSize(),
                     req.transactionNodes(),
                     req.subjectId(),


[07/10] incubator-ignite git commit: ignite-484-1 - improved retry

Posted by se...@apache.org.
ignite-484-1 - improved retry


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

Branch: refs/heads/ignite-484-1
Commit: 94060c9ef41161c7262a28044ddb176f86814b01
Parents: 10febf2
Author: S.Vladykin <sv...@gridgain.com>
Authored: Wed Jun 17 19:46:42 2015 +0300
Committer: S.Vladykin <sv...@gridgain.com>
Committed: Wed Jun 17 19:46:42 2015 +0300

----------------------------------------------------------------------
 .../query/h2/twostep/GridMapQueryExecutor.java  | 26 ++++--
 .../h2/twostep/GridReduceQueryExecutor.java     | 86 ++++++++++++++------
 ...lientQueryReplicatedNodeRestartSelfTest.java | 50 ++++++++++--
 3 files changed, 125 insertions(+), 37 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/94060c9e/modules/indexing/src/main/java/org/apache/ignite/internal/processors/query/h2/twostep/GridMapQueryExecutor.java
----------------------------------------------------------------------
diff --git a/modules/indexing/src/main/java/org/apache/ignite/internal/processors/query/h2/twostep/GridMapQueryExecutor.java b/modules/indexing/src/main/java/org/apache/ignite/internal/processors/query/h2/twostep/GridMapQueryExecutor.java
index aaf64ee..2503a87 100644
--- a/modules/indexing/src/main/java/org/apache/ignite/internal/processors/query/h2/twostep/GridMapQueryExecutor.java
+++ b/modules/indexing/src/main/java/org/apache/ignite/internal/processors/query/h2/twostep/GridMapQueryExecutor.java
@@ -48,6 +48,7 @@ import java.util.concurrent.*;
 import java.util.concurrent.atomic.*;
 
 import static org.apache.ignite.events.EventType.*;
+import static org.apache.ignite.internal.processors.affinity.AffinityTopologyVersion.*;
 import static org.apache.ignite.internal.processors.cache.distributed.dht.GridDhtPartitionState.*;
 import static org.apache.ignite.internal.processors.query.h2.twostep.msg.GridH2ValueMessageFactory.*;
 
@@ -230,6 +231,15 @@ public class GridMapQueryExecutor {
     }
 
     /**
+     * @param cctx Cache context.
+     * @param p Partition ID.
+     * @return Partition.
+     */
+    private GridDhtLocalPartition partition(GridCacheContext<?, ?> cctx, int p) {
+        return cctx.topology().localPartition(p, NONE, false);
+    }
+
+    /**
      * @param cacheNames Cache names.
      * @param topVer Topology version.
      * @param explicitParts Explicit partitions list.
@@ -263,10 +273,12 @@ public class GridMapQueryExecutor {
             GridReservable r = reservations.get(grpKey);
 
             if (explicitParts == null && r != null) { // Try to reserve group partition if any and no explicits.
-                if (!r.reserve())
-                    return false; // We need explicit partitions here -> retry.
+                if (r != ReplicatedReservation.INSTANCE) {
+                    if (!r.reserve())
+                        return false; // We need explicit partitions here -> retry.
 
-                reserved.add(r);
+                    reserved.add(r);
+                }
             }
             else { // Try to reserve partitions one by one.
                 int partsCnt = cctx.affinity().partitions();
@@ -274,7 +286,7 @@ public class GridMapQueryExecutor {
                 if (cctx.isReplicated()) { // Check all the partitions are in owning state for replicated cache.
                     if (r == null) { // Check only once.
                         for (int p = 0; p < partsCnt; p++) {
-                            GridDhtLocalPartition part = cctx.topology().localPartition(p, topVer, false);
+                            GridDhtLocalPartition part = partition(cctx, p);
 
                             // We don't need to reserve partitions because they will not be evicted in replicated caches.
                             if (part == null || part.state() != OWNING)
@@ -290,7 +302,7 @@ public class GridMapQueryExecutor {
                         partIds = cctx.affinity().primaryPartitions(ctx.localNodeId(), topVer);
 
                     for (int partId : partIds) {
-                        GridDhtLocalPartition part = cctx.topology().localPartition(partId, topVer, false);
+                        GridDhtLocalPartition part = partition(cctx, partId);
 
                         if (part == null || part.state() != OWNING || !part.reserve())
                             return false;
@@ -806,12 +818,12 @@ public class GridMapQueryExecutor {
 
         /** {@inheritDoc} */
         @Override public boolean reserve() {
-            return true;
+            throw new IllegalStateException();
         }
 
         /** {@inheritDoc} */
         @Override public void release() {
-            // No-op.
+            throw new IllegalStateException();
         }
     }
 }

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/94060c9e/modules/indexing/src/main/java/org/apache/ignite/internal/processors/query/h2/twostep/GridReduceQueryExecutor.java
----------------------------------------------------------------------
diff --git a/modules/indexing/src/main/java/org/apache/ignite/internal/processors/query/h2/twostep/GridReduceQueryExecutor.java b/modules/indexing/src/main/java/org/apache/ignite/internal/processors/query/h2/twostep/GridReduceQueryExecutor.java
index c570d24..6635dde 100644
--- a/modules/indexing/src/main/java/org/apache/ignite/internal/processors/query/h2/twostep/GridReduceQueryExecutor.java
+++ b/modules/indexing/src/main/java/org/apache/ignite/internal/processors/query/h2/twostep/GridReduceQueryExecutor.java
@@ -335,7 +335,7 @@ public class GridReduceQueryExecutor {
     ) {
         String space = cctx.name();
 
-        Set<ClusterNode> nodes = new HashSet<>(ctx.discovery().cacheAffinityNodes(space, topVer));
+        Set<ClusterNode> nodes = new HashSet<>(dataNodes(space, topVer));
 
         if (F.isEmpty(nodes))
             throw new CacheException("No data nodes found for cache: " + space);
@@ -351,7 +351,7 @@ public class GridReduceQueryExecutor {
                     throw new CacheException("Queries running on replicated cache should not contain JOINs " +
                         "with partitioned tables.");
 
-                Collection<ClusterNode> extraNodes = ctx.discovery().cacheAffinityNodes(extraSpace, topVer);
+                Collection<ClusterNode> extraNodes = dataNodes(extraSpace, topVer);
 
                 if (F.isEmpty(extraNodes))
                     throw new CacheException("No data nodes found for cache: " + extraSpace);
@@ -398,7 +398,18 @@ public class GridReduceQueryExecutor {
      * @return Cursor.
      */
     public Iterator<List<?>> query(GridCacheContext<?,?> cctx, GridCacheTwoStepQuery qry, boolean keepPortable) {
-        for (;;) {
+        for (int attempt = 0;; attempt++) {
+            if (attempt != 0) {
+                try {
+                    Thread.sleep(attempt * 10); // Wait for exchange.
+                }
+                catch (InterruptedException e) {
+                    Thread.currentThread().interrupt();
+
+                    throw new CacheException("Query was interrupted.", e);
+                }
+            }
+
             long qryReqId = reqIdGen.incrementAndGet();
 
             QueryRun r = new QueryRun();
@@ -422,9 +433,9 @@ public class GridReduceQueryExecutor {
 
             if (isPreloadingActive(cctx, extraSpaces)) {
                 if (cctx.isReplicated())
-                    nodes = replicatedDataNodes(cctx, extraSpaces);
+                    nodes = replicatedUnstableDataNodes(cctx, extraSpaces);
                 else {
-                    partsMap = partitionLocations(cctx, extraSpaces);
+                    partsMap = partitionedUnstableDataNodes(cctx, extraSpaces);
 
                     nodes = partsMap == null ? null : partsMap.keySet();
                 }
@@ -538,9 +549,6 @@ public class GridReduceQueryExecutor {
             catch (IgniteCheckedException | RuntimeException e) {
                 U.closeQuiet(r.conn);
 
-                if (e instanceof CacheException)
-                    throw (CacheException)e;
-
                 throw new CacheException("Failed to run reduce query locally.", e);
             }
             finally {
@@ -559,10 +567,14 @@ public class GridReduceQueryExecutor {
      * @param extraSpaces Extra spaces.
      * @return Collection of all data nodes owning all the caches or {@code null} for retry.
      */
-    private Collection<ClusterNode> replicatedDataNodes(final GridCacheContext<?,?> cctx, List<String> extraSpaces) {
+    private Collection<ClusterNode> replicatedUnstableDataNodes(final GridCacheContext<?,?> cctx,
+        List<String> extraSpaces) {
         assert cctx.isReplicated() : cctx.name() + " must be replicated";
 
-        Set<ClusterNode> nodes = owningReplicatedDataNodes(cctx);
+        Set<ClusterNode> nodes = replicatedUnstableDataNodes(cctx);
+
+        if (F.isEmpty(nodes))
+            return null; // Retry.
 
         if (!F.isEmpty(extraSpaces)) {
             for (String extraSpace : extraSpaces) {
@@ -575,7 +587,12 @@ public class GridReduceQueryExecutor {
                     throw new CacheException("Queries running on replicated cache should not contain JOINs " +
                         "with partitioned tables.");
 
-                nodes.retainAll(owningReplicatedDataNodes(extraCctx));
+                Set<ClusterNode> extraOwners = replicatedUnstableDataNodes(extraCctx);
+
+                if (F.isEmpty(extraOwners))
+                    return null; // Retry.
+
+                nodes.retainAll(extraOwners);
 
                 if (nodes.isEmpty())
                     return null; // Retry.
@@ -586,34 +603,43 @@ public class GridReduceQueryExecutor {
     }
 
     /**
+     * @param space Cache name.
+     * @param topVer Topology version.
+     * @return Collection of data nodes.
+     */
+    private Collection<ClusterNode> dataNodes(String space, AffinityTopologyVersion topVer) {
+        Collection<ClusterNode> res = ctx.discovery().cacheAffinityNodes(space, topVer);
+
+        return res != null ? res : Collections.<ClusterNode>emptySet();
+    }
+
+    /**
      * Collects all the nodes owning all the partitions for the given replicated cache.
      *
      * @param cctx Cache context.
-     * @return Owning nodes.
+     * @return Owning nodes or {@code null} if we can't find owners for some partitions.
      */
-    private Set<ClusterNode> owningReplicatedDataNodes(GridCacheContext<?,?> cctx) {
+    private Set<ClusterNode> replicatedUnstableDataNodes(GridCacheContext<?,?> cctx) {
         assert cctx.isReplicated() : cctx.name() + " must be replicated";
 
         String space = cctx.name();
 
-        Set<ClusterNode> dataNodes = new HashSet<>(ctx.discovery().cacheAffinityNodes(space, NONE));
+        Set<ClusterNode> dataNodes = new HashSet<>(dataNodes(space, NONE));
 
         if (dataNodes.isEmpty())
             throw new CacheException("No data nodes found for cache '" + space + "'");
 
         // Find all the nodes owning all the partitions for replicated cache.
-        for (int p = 0, extraParts = cctx.affinity().partitions(); p < extraParts; p++) {
+        for (int p = 0, parts = cctx.affinity().partitions(); p < parts; p++) {
             List<ClusterNode> owners = cctx.topology().owners(p);
 
-            if (owners.isEmpty())
-                throw new CacheException("No data nodes found for cache '" + space +
-                    "' for partition " + p);
+            if (F.isEmpty(owners))
+                return null; // Retry.
 
             dataNodes.retainAll(owners);
 
             if (dataNodes.isEmpty())
-                throw new CacheException("No data nodes found for cache '" + space +
-                    "' owning all the partitions.");
+                return null; // Retry.
         }
 
         return dataNodes;
@@ -627,7 +653,8 @@ public class GridReduceQueryExecutor {
      * @return Partition mapping or {@code null} if we can't calculate it due to repartitioning and we need to retry.
      */
     @SuppressWarnings("unchecked")
-    private Map<ClusterNode, IntArray> partitionLocations(final GridCacheContext<?,?> cctx, List<String> extraSpaces) {
+    private Map<ClusterNode, IntArray> partitionedUnstableDataNodes(final GridCacheContext<?,?> cctx,
+        List<String> extraSpaces) {
         assert !cctx.isReplicated() && !cctx.isLocal() : cctx.name() + " must be partitioned";
 
         final int partsCnt = cctx.affinity().partitions();
@@ -653,8 +680,12 @@ public class GridReduceQueryExecutor {
         for (int p = 0, parts =  cctx.affinity().partitions(); p < parts; p++) {
             List<ClusterNode> owners = cctx.topology().owners(p);
 
-            if (F.isEmpty(owners))
+            if (F.isEmpty(owners)) {
+                if (!F.isEmpty(dataNodes(cctx.name(), NONE)))
+                    return null; // Retry.
+
                 throw new CacheException("No data nodes found for cache '" + cctx.name() + "' for partition " + p);
+            }
 
             partLocs[p] = new HashSet<>(owners);
         }
@@ -671,9 +702,13 @@ public class GridReduceQueryExecutor {
                 for (int p = 0, parts =  extraCctx.affinity().partitions(); p < parts; p++) {
                     List<ClusterNode> owners = extraCctx.topology().owners(p);
 
-                    if (F.isEmpty(owners))
+                    if (F.isEmpty(owners)) {
+                        if (!F.isEmpty(dataNodes(extraSpace, NONE)))
+                            return null; // Retry.
+
                         throw new CacheException("No data nodes found for cache '" + extraSpace +
                             "' for partition " + p);
+                    }
 
                     if (partLocs[p] == null)
                         partLocs[p] = new HashSet<>(owners);
@@ -693,7 +728,10 @@ public class GridReduceQueryExecutor {
                 if (!extraCctx.isReplicated())
                     continue;
 
-                Set<ClusterNode> dataNodes = owningReplicatedDataNodes(extraCctx);
+                Set<ClusterNode> dataNodes = replicatedUnstableDataNodes(extraCctx);
+
+                if (F.isEmpty(dataNodes))
+                    return null; // Retry.
 
                 for (Set<ClusterNode> partLoc : partLocs) {
                     partLoc.retainAll(dataNodes);

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/94060c9e/modules/indexing/src/test/java/org/apache/ignite/internal/processors/cache/distributed/near/IgniteCacheClientQueryReplicatedNodeRestartSelfTest.java
----------------------------------------------------------------------
diff --git a/modules/indexing/src/test/java/org/apache/ignite/internal/processors/cache/distributed/near/IgniteCacheClientQueryReplicatedNodeRestartSelfTest.java b/modules/indexing/src/test/java/org/apache/ignite/internal/processors/cache/distributed/near/IgniteCacheClientQueryReplicatedNodeRestartSelfTest.java
index 23f44c0..3f23005 100644
--- a/modules/indexing/src/test/java/org/apache/ignite/internal/processors/cache/distributed/near/IgniteCacheClientQueryReplicatedNodeRestartSelfTest.java
+++ b/modules/indexing/src/test/java/org/apache/ignite/internal/processors/cache/distributed/near/IgniteCacheClientQueryReplicatedNodeRestartSelfTest.java
@@ -64,6 +64,9 @@ public class IgniteCacheClientQueryReplicatedNodeRestartSelfTest extends GridCom
         };
 
     /** */
+    private static final List<List<?>> FAKE = new LinkedList<>();
+
+    /** */
     private static final int GRID_CNT = 5;
 
     /** */
@@ -191,7 +194,7 @@ public class IgniteCacheClientQueryReplicatedNodeRestartSelfTest extends GridCom
     public void testRestarts() throws Exception {
         int duration = 90 * 1000;
         int qryThreadNum = 5;
-        int restartThreadsNum = 2; // 2 of 4 data nodes
+        int restartThreadsNum = 3; // 3 of 4 data nodes
         final int nodeLifeTime = 2 * 1000;
         final int logFreq = 10;
 
@@ -212,13 +215,32 @@ public class IgniteCacheClientQueryReplicatedNodeRestartSelfTest extends GridCom
         final AtomicInteger qryCnt = new AtomicInteger();
         final AtomicBoolean qrysDone = new AtomicBoolean();
 
+        final List<Integer> cacheSize = new ArrayList<>(4);
+
         for (int i = 0; i < GRID_CNT - 1; i++) {
-            for (String cacheName : F.asList("co", "pr", "pe", "pu"))
-                assertClient(grid(i).cache(cacheName), false);
+            int j = 0;
+
+            for (String cacheName : F.asList("co", "pr", "pe", "pu")) {
+                IgniteCache<?,?> cache = grid(i).cache(cacheName);
+
+                assertClient(cache, false);
+
+                if (i == 0)
+                    cacheSize.add(cache.size());
+                else
+                    assertEquals(cacheSize.get(j++).intValue(), cache.size());
+            }
         }
 
-        for (String cacheName : F.asList("co", "pr", "pe", "pu"))
-            assertClient(grid(GRID_CNT - 1).cache(cacheName), true);
+        int j = 0;
+
+        for (String cacheName : F.asList("co", "pr", "pe", "pu")) {
+            IgniteCache<?,?> cache = grid(GRID_CNT - 1).cache(cacheName);
+
+            assertClient(cache, true);
+
+            assertEquals(cacheSize.get(j++).intValue(), cache.size());
+        }
 
         final IgniteCache<?,?> clientCache = grid(GRID_CNT - 1).cache("pu");
 
@@ -234,8 +256,10 @@ public class IgniteCacheClientQueryReplicatedNodeRestartSelfTest extends GridCom
                     if (smallPageSize)
                         qry.setPageSize(3);
 
+                    List<List<?>> res;
+
                     try {
-                        assertEquals(pRes, clientCache.query(qry).getAll());
+                        res = clientCache.query(qry).getAll();
                     }
                     catch (CacheException e) {
                         assertTrue("On large page size must retry.", smallPageSize);
@@ -259,6 +283,20 @@ public class IgniteCacheClientQueryReplicatedNodeRestartSelfTest extends GridCom
 
                             fail("Must fail inside of GridResultPage.fetchNextPage or subclass.");
                         }
+
+                        res = FAKE;
+                    }
+
+                    if (res != FAKE && !res.equals(pRes)) {
+                        int j = 0;
+
+                        // Check for data loss.
+                        for (String cacheName : F.asList("co", "pr", "pe", "pu")) {
+                            assertEquals(cacheName, cacheSize.get(j++).intValue(),
+                                grid(GRID_CNT - 1).cache(cacheName).size());
+                        }
+
+                        assertEquals(pRes, res); // Fail with nice message.
                     }
 
                     int c = qryCnt.incrementAndGet();


[09/10] incubator-ignite git commit: ignite-484-1 - error messages

Posted by se...@apache.org.
ignite-484-1 - error messages


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

Branch: refs/heads/ignite-484-1
Commit: e602ca01e48fd43f4cc33b729078c105d84601f6
Parents: efb4244
Author: S.Vladykin <sv...@gridgain.com>
Authored: Wed Jun 17 20:15:45 2015 +0300
Committer: S.Vladykin <sv...@gridgain.com>
Committed: Wed Jun 17 20:15:45 2015 +0300

----------------------------------------------------------------------
 .../query/h2/twostep/GridMapQueryExecutor.java  |  4 +--
 .../h2/twostep/GridReduceQueryExecutor.java     | 31 ++++++++++----------
 2 files changed, 17 insertions(+), 18 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/e602ca01/modules/indexing/src/main/java/org/apache/ignite/internal/processors/query/h2/twostep/GridMapQueryExecutor.java
----------------------------------------------------------------------
diff --git a/modules/indexing/src/main/java/org/apache/ignite/internal/processors/query/h2/twostep/GridMapQueryExecutor.java b/modules/indexing/src/main/java/org/apache/ignite/internal/processors/query/h2/twostep/GridMapQueryExecutor.java
index 2503a87..ba95d43 100644
--- a/modules/indexing/src/main/java/org/apache/ignite/internal/processors/query/h2/twostep/GridMapQueryExecutor.java
+++ b/modules/indexing/src/main/java/org/apache/ignite/internal/processors/query/h2/twostep/GridMapQueryExecutor.java
@@ -401,7 +401,7 @@ public class GridMapQueryExecutor {
                 }
             }
             catch (IgniteCheckedException e) {
-                throw new CacheException(e);
+                throw new CacheException("Failed to unmarshall parameters.", e);
             }
 
             List<String> caches = (List<String>)F.concat(true, req.space(), req.extraSpaces());
@@ -422,7 +422,7 @@ public class GridMapQueryExecutor {
             GridCacheContext<?,?> mainCctx = cacheContext(req.space());
 
             if (mainCctx == null)
-                throw new CacheException("Cache was destroyed: " + req.space());
+                throw new CacheException("Failed to find cache: " + req.space());
 
             qr = new QueryResults(req.requestId(), qrys.size(), mainCctx);
 

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/e602ca01/modules/indexing/src/main/java/org/apache/ignite/internal/processors/query/h2/twostep/GridReduceQueryExecutor.java
----------------------------------------------------------------------
diff --git a/modules/indexing/src/main/java/org/apache/ignite/internal/processors/query/h2/twostep/GridReduceQueryExecutor.java b/modules/indexing/src/main/java/org/apache/ignite/internal/processors/query/h2/twostep/GridReduceQueryExecutor.java
index b956167..32d1c95 100644
--- a/modules/indexing/src/main/java/org/apache/ignite/internal/processors/query/h2/twostep/GridReduceQueryExecutor.java
+++ b/modules/indexing/src/main/java/org/apache/ignite/internal/processors/query/h2/twostep/GridReduceQueryExecutor.java
@@ -338,7 +338,7 @@ public class GridReduceQueryExecutor {
         Set<ClusterNode> nodes = new HashSet<>(dataNodes(space, topVer));
 
         if (F.isEmpty(nodes))
-            throw new CacheException("No data nodes found for cache: " + space);
+            throw new CacheException("Failed to find data nodes for cache: " + space);
 
         if (!F.isEmpty(extraSpaces)) {
             for (String extraSpace : extraSpaces) {
@@ -349,12 +349,12 @@ public class GridReduceQueryExecutor {
 
                 if (cctx.isReplicated() && !extraCctx.isReplicated())
                     throw new CacheException("Queries running on replicated cache should not contain JOINs " +
-                        "with partitioned tables.");
+                        "with partitioned tables [rCache=" + cctx.name() + ", pCache=" + extraSpace + "]");
 
                 Collection<ClusterNode> extraNodes = dataNodes(extraSpace, topVer);
 
                 if (F.isEmpty(extraNodes))
-                    throw new CacheException("No data nodes found for cache: " + extraSpace);
+                    throw new CacheException("Failed to find data nodes for cache: " + extraSpace);
 
                 if (cctx.isReplicated() && extraCctx.isReplicated()) {
                     nodes.retainAll(extraNodes);
@@ -363,8 +363,8 @@ public class GridReduceQueryExecutor {
                         if (isPreloadingActive(cctx, extraSpaces))
                             return null; // Retry.
                         else
-                            throw new CacheException("Caches '" + cctx.name() + "' and '" + extraSpace +
-                                "' have distinct set of data nodes.");
+                            throw new CacheException("Caches have distinct sets of data nodes [cache1=" + cctx.name() +
+                                ", cache2=" + extraSpace + "]");
                     }
                 }
                 else if (!cctx.isReplicated() && extraCctx.isReplicated()) {
@@ -372,16 +372,16 @@ public class GridReduceQueryExecutor {
                         if (isPreloadingActive(cctx, extraSpaces))
                             return null; // Retry.
                         else
-                            throw new CacheException("Caches '" + cctx.name() + "' and '" + extraSpace +
-                                "' have distinct set of data nodes.");
+                            throw new CacheException("Caches have distinct sets of data nodes [cache1=" + cctx.name() +
+                                ", cache2=" + extraSpace + "]");
                 }
                 else if (!cctx.isReplicated() && !extraCctx.isReplicated()) {
                     if (extraNodes.size() != nodes.size() || !nodes.containsAll(extraNodes))
                         if (isPreloadingActive(cctx, extraSpaces))
                             return null; // Retry.
                         else
-                            throw new CacheException("Caches '" + cctx.name() + "' and '" + extraSpace +
-                                "' have distinct set of data nodes.");
+                            throw new CacheException("Caches have distinct sets of data nodes [cache1=" + cctx.name() +
+                                ", cache2=" + extraSpace + "]");
                 }
                 else
                     throw new IllegalStateException();
@@ -586,7 +586,7 @@ public class GridReduceQueryExecutor {
 
                 if (!extraCctx.isReplicated())
                     throw new CacheException("Queries running on replicated cache should not contain JOINs " +
-                        "with partitioned tables.");
+                        "with tables in partitioned caches [rCache=" + cctx.name() + ", pCache=" + extraSpace + "]");
 
                 Set<ClusterNode> extraOwners = replicatedUnstableDataNodes(extraCctx);
 
@@ -628,7 +628,7 @@ public class GridReduceQueryExecutor {
         Set<ClusterNode> dataNodes = new HashSet<>(dataNodes(space, NONE));
 
         if (dataNodes.isEmpty())
-            throw new CacheException("No data nodes found for cache '" + space + "'");
+            throw new CacheException("Failed to find data nodes for cache: " + space);
 
         // Find all the nodes owning all the partitions for replicated cache.
         for (int p = 0, parts = cctx.affinity().partitions(); p < parts; p++) {
@@ -670,8 +670,8 @@ public class GridReduceQueryExecutor {
                 int parts = extraCctx.affinity().partitions();
 
                 if (parts != partsCnt)
-                    throw new CacheException("Number of partitions must be the same for correct collocation in " +
-                        "caches " + cctx.name() + " and " + extraSpace + ".");
+                    throw new CacheException("Number of partitions must be the same for correct collocation [cache1=" +
+                        cctx.name() + ", parts1=" + partsCnt + ", cache2=" + extraSpace + ", parts2=" + parts + "]");
             }
         }
 
@@ -685,7 +685,7 @@ public class GridReduceQueryExecutor {
                 if (!F.isEmpty(dataNodes(cctx.name(), NONE)))
                     return null; // Retry.
 
-                throw new CacheException("No data nodes found for cache '" + cctx.name() + "' for partition " + p);
+                throw new CacheException("Failed to find data nodes [cache=" + cctx.name() + ", part=" + p + "]");
             }
 
             partLocs[p] = new HashSet<>(owners);
@@ -707,8 +707,7 @@ public class GridReduceQueryExecutor {
                         if (!F.isEmpty(dataNodes(extraSpace, NONE)))
                             return null; // Retry.
 
-                        throw new CacheException("No data nodes found for cache '" + extraSpace +
-                            "' for partition " + p);
+                        throw new CacheException("Failed to find data nodes [cache=" + extraSpace + ", part=" + p + "]");
                     }
 
                     if (partLocs[p] == null)


[04/10] incubator-ignite git commit: Merge remote-tracking branch 'remotes/origin/ignite-sprint-7' into ignite-1020

Posted by se...@apache.org.
Merge remote-tracking branch 'remotes/origin/ignite-sprint-7' into ignite-1020


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

Branch: refs/heads/ignite-484-1
Commit: e507f030998c98851f97affa14667b64ebdb236e
Parents: a75b868 01eee2d
Author: sboikov <sb...@gridgain.com>
Authored: Wed Jun 17 12:10:41 2015 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Wed Jun 17 12:10:41 2015 +0300

----------------------------------------------------------------------
 .../apache/ignite/internal/IgniteKernal.java    | 13 +-----------
 .../client/GridClientConfiguration.java         |  4 ++--
 .../GridClientOptimizedMarshaller.java          | 21 ++++++++++++++++++++
 .../impl/GridTcpRouterNioListenerAdapter.java   |  2 +-
 .../rest/protocols/tcp/GridTcpRestProtocol.java |  3 ++-
 .../ignite/internal/util/IgniteUtils.java       | 21 ++++++++++++++++++++
 6 files changed, 48 insertions(+), 16 deletions(-)
----------------------------------------------------------------------



[05/10] incubator-ignite git commit: # ignite-1020 do not write to store for local store load

Posted by se...@apache.org.
# ignite-1020 do not write to store for local store load


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

Branch: refs/heads/ignite-484-1
Commit: 674072a336a2aa276373a5f6820a9e66df88ede8
Parents: e507f03
Author: sboikov <sb...@gridgain.com>
Authored: Wed Jun 17 15:31:57 2015 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Wed Jun 17 15:31:57 2015 +0300

----------------------------------------------------------------------
 .../ignite/internal/processors/cache/GridCacheAdapter.java    | 4 ++++
 .../ignite/internal/processors/cache/IgniteCacheProxy.java    | 7 +++++++
 .../processors/dr/IgniteDrDataStreamerCacheUpdater.java       | 7 ++++++-
 3 files changed, 17 insertions(+), 1 deletion(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/674072a3/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 2ca7687..7335d72 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
@@ -3305,6 +3305,8 @@ public abstract class GridCacheAdapter<K, V> implements IgniteInternalCache<K, V
                 DataStreamerImpl ldr = ctx.kernalContext().dataStream().dataStreamer(ctx.namex());
 
                 try {
+                    ldr.skipStore(true);
+
                     ldr.receiver(new IgniteDrDataStreamerCacheUpdater());
 
                     LocalStoreLoadClosure c = new LocalStoreLoadClosure(p, ldr, plc);
@@ -3504,6 +3506,8 @@ public abstract class GridCacheAdapter<K, V> implements IgniteInternalCache<K, V
             DataStreamerImpl ldr = ctx.kernalContext().dataStream().dataStreamer(ctx.namex());
 
             try {
+                ldr.skipStore(true);
+
                 ldr.receiver(new IgniteDrDataStreamerCacheUpdater());
 
                 LocalStoreLoadClosure c = new LocalStoreLoadClosure(null, ldr, plc0);

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/674072a3/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/IgniteCacheProxy.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/IgniteCacheProxy.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/IgniteCacheProxy.java
index 69ce7b0..0d70792 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/IgniteCacheProxy.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/IgniteCacheProxy.java
@@ -139,6 +139,13 @@ public class IgniteCacheProxy<K, V> extends AsyncSupportAdapter<IgniteCache<K, V
     }
 
     /**
+     * @return Operation context.
+     */
+    @Nullable public CacheOperationContext operationContext() {
+        return opCtx;
+    }
+
+    /**
      * Gets cache proxy which does not acquire read lock on gateway enter, should be
      * used only if grid read lock is externally acquired.
      *

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/674072a3/modules/core/src/main/java/org/apache/ignite/internal/processors/dr/IgniteDrDataStreamerCacheUpdater.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/dr/IgniteDrDataStreamerCacheUpdater.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/dr/IgniteDrDataStreamerCacheUpdater.java
index c3a8989..e5bbe39 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/dr/IgniteDrDataStreamerCacheUpdater.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/dr/IgniteDrDataStreamerCacheUpdater.java
@@ -46,7 +46,12 @@ public class IgniteDrDataStreamerCacheUpdater implements StreamReceiver<KeyCache
 
             GridKernalContext ctx = ((IgniteKernal)cache0.unwrap(Ignite.class)).context();
             IgniteLogger log = ctx.log(IgniteDrDataStreamerCacheUpdater.class);
-            GridCacheAdapter cache = ctx.cache().internalCache(cacheName);
+            GridCacheAdapter internalCache = ctx.cache().internalCache(cacheName);
+
+            CacheOperationContext opCtx = ((IgniteCacheProxy)cache0).operationContext();
+
+            IgniteInternalCache cache =
+                opCtx != null ? new GridCacheProxyImpl(internalCache.context(), internalCache, opCtx) : internalCache;
 
             assert !F.isEmpty(col);
 


[08/10] incubator-ignite git commit: Merge branches 'ignite-484-1' and 'ignite-sprint-6' of https://git-wip-us.apache.org/repos/asf/incubator-ignite into ignite-484-1

Posted by se...@apache.org.
Merge branches 'ignite-484-1' and 'ignite-sprint-6' of https://git-wip-us.apache.org/repos/asf/incubator-ignite into ignite-484-1


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

Branch: refs/heads/ignite-484-1
Commit: efb4244779b94c9c9f35c63708e4a41da2430bce
Parents: 94060c9 4298238 af829d0
Author: S.Vladykin <sv...@gridgain.com>
Authored: Wed Jun 17 19:50:12 2015 +0300
Committer: S.Vladykin <sv...@gridgain.com>
Committed: Wed Jun 17 19:50:12 2015 +0300

----------------------------------------------------------------------
 .../processors/cache/GridCacheAdapter.java      |   4 +
 .../processors/cache/IgniteCacheProxy.java      |   7 +
 .../dht/GridDhtTransactionalCacheAdapter.java   |   2 +-
 .../cache/transactions/IgniteTxHandler.java     |   2 +-
 .../transactions/IgniteTxLocalAdapter.java      |  12 +-
 .../dr/IgniteDrDataStreamerCacheUpdater.java    |   7 +-
 .../CacheStoreUsageMultinodeAbstractTest.java   | 305 +++++++++++++++++++
 ...eUsageMultinodeDynamicStartAbstractTest.java | 169 ++++++++++
 ...oreUsageMultinodeDynamicStartAtomicTest.java |  32 ++
 ...heStoreUsageMultinodeDynamicStartTxTest.java |  32 ++
 ...reUsageMultinodeStaticStartAbstractTest.java | 158 ++++++++++
 ...toreUsageMultinodeStaticStartAtomicTest.java |  32 ++
 ...cheStoreUsageMultinodeStaticStartTxTest.java |  32 ++
 .../testsuites/IgniteCacheTestSuite4.java       |   4 +
 .../h2/twostep/GridReduceQueryExecutor.java     |   3 +-
 15 files changed, 793 insertions(+), 8 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/efb42447/modules/indexing/src/main/java/org/apache/ignite/internal/processors/query/h2/twostep/GridReduceQueryExecutor.java
----------------------------------------------------------------------
diff --cc modules/indexing/src/main/java/org/apache/ignite/internal/processors/query/h2/twostep/GridReduceQueryExecutor.java
index 6635dde,6c407d9,11054b7..b956167
--- a/modules/indexing/src/main/java/org/apache/ignite/internal/processors/query/h2/twostep/GridReduceQueryExecutor.java
+++ b/modules/indexing/src/main/java/org/apache/ignite/internal/processors/query/h2/twostep/GridReduceQueryExecutor.java
@@@@ -273,514 -273,477 -265,113 +273,515 @@@@ public class GridReduceQueryExecutor 
       }
   
       /**
  +     * @param r Query run.
  +     * @param retryVer Retry version.
  +     * @param nodeId Node ID.
  +     */
  +    private void retry(QueryRun r, AffinityTopologyVersion retryVer, UUID nodeId) {
  +        r.state(retryVer, nodeId);
  +    }
  +
  +    /**
  +     * @param cctx Cache context for main space.
  +     * @param extraSpaces Extra spaces.
  +     * @return {@code true} If preloading is active.
  +     */
  +    private boolean isPreloadingActive(final GridCacheContext<?,?> cctx, List<String> extraSpaces) {
  +        if (hasMovingPartitions(cctx))
  +            return true;
  +
  +        if (extraSpaces != null) {
  +            for (String extraSpace : extraSpaces) {
  +                if (hasMovingPartitions(cacheContext(extraSpace)))
  +                    return true;
  +            }
  +        }
  +
  +        return false;
  +    }
  +
  +    /**
  +     * @return {@code true} If cache context
  +     */
  +    private boolean hasMovingPartitions(GridCacheContext<?,?> cctx) {
  +        GridDhtPartitionFullMap fullMap = cctx.topology().partitionMap(false);
  +
  +        for (GridDhtPartitionMap map : fullMap.values()) {
  +            if (map.hasMovingPartitions())
  +                return true;
  +        }
  +
  +        return false;
  +    }
  +
  +    /**
  +     * @param name Cache name.
  +     * @return Cache context.
  +     */
  +    private GridCacheContext<?,?> cacheContext(String name) {
  +        return ctx.cache().internalCache(name).context();
  +    }
  +
  +    /**
  +     * @param topVer Topology version.
  +     * @param cctx Cache context for main space.
  +     * @param extraSpaces Extra spaces.
  +     * @return Data nodes or {@code null} if repartitioning started and we need to retry..
  +     */
  +    private Collection<ClusterNode> stableDataNodes(
  +        AffinityTopologyVersion topVer,
  +        final GridCacheContext<?,?> cctx,
  +        List<String> extraSpaces
  +    ) {
  +        String space = cctx.name();
  +
 -         Set<ClusterNode> nodes = new HashSet<>(ctx.discovery().cacheAffinityNodes(space, topVer));
 ++        Set<ClusterNode> nodes = new HashSet<>(dataNodes(space, topVer));
  +
  +        if (F.isEmpty(nodes))
  +            throw new CacheException("No data nodes found for cache: " + space);
  +
  +        if (!F.isEmpty(extraSpaces)) {
  +            for (String extraSpace : extraSpaces) {
  +                GridCacheContext<?,?> extraCctx = cacheContext(extraSpace);
  +
  +                if (extraCctx.isLocal())
  +                    continue; // No consistency guaranties for local caches.
  +
  +                if (cctx.isReplicated() && !extraCctx.isReplicated())
  +                    throw new CacheException("Queries running on replicated cache should not contain JOINs " +
  +                        "with partitioned tables.");
  +
 -                 Collection<ClusterNode> extraNodes = ctx.discovery().cacheAffinityNodes(extraSpace, topVer);
 ++                Collection<ClusterNode> extraNodes = dataNodes(extraSpace, topVer);
  +
  +                if (F.isEmpty(extraNodes))
  +                    throw new CacheException("No data nodes found for cache: " + extraSpace);
  +
  +                if (cctx.isReplicated() && extraCctx.isReplicated()) {
  +                    nodes.retainAll(extraNodes);
  +
  +                    if (nodes.isEmpty()) {
  +                        if (isPreloadingActive(cctx, extraSpaces))
  +                            return null; // Retry.
  +                        else
  +                            throw new CacheException("Caches '" + cctx.name() + "' and '" + extraSpace +
  +                                "' have distinct set of data nodes.");
  +                    }
  +                }
  +                else if (!cctx.isReplicated() && extraCctx.isReplicated()) {
  +                    if (!extraNodes.containsAll(nodes))
  +                        if (isPreloadingActive(cctx, extraSpaces))
  +                            return null; // Retry.
  +                        else
  +                            throw new CacheException("Caches '" + cctx.name() + "' and '" + extraSpace +
  +                                "' have distinct set of data nodes.");
  +                }
  +                else if (!cctx.isReplicated() && !extraCctx.isReplicated()) {
  +                    if (extraNodes.size() != nodes.size() || !nodes.containsAll(extraNodes))
  +                        if (isPreloadingActive(cctx, extraSpaces))
  +                            return null; // Retry.
  +                        else
  +                            throw new CacheException("Caches '" + cctx.name() + "' and '" + extraSpace +
  +                                "' have distinct set of data nodes.");
  +                }
  +                else
  +                    throw new IllegalStateException();
  +            }
  +        }
  +
  +        return nodes;
  +    }
  +
  +    /**
        * @param cctx Cache context.
        * @param qry Query.
  +     * @param keepPortable Keep portable.
        * @return Cursor.
        */
       public Iterator<List<?>> query(GridCacheContext<?,?> cctx, GridCacheTwoStepQuery qry, boolean keepPortable) {
 -         for (;;) {
  -        long qryReqId = reqIdGen.incrementAndGet();
 ++        for (int attempt = 0;; attempt++) {
 ++            if (attempt != 0) {
 ++                try {
 ++                    Thread.sleep(attempt * 10); // Wait for exchange.
 ++                }
 ++                catch (InterruptedException e) {
 ++                    Thread.currentThread().interrupt();
 + 
  -        QueryRun r = new QueryRun();
 ++                    throw new CacheException("Query was interrupted.", e);
 ++                }
 ++            }
 + 
  -        r.pageSize = qry.pageSize() <= 0 ? GridCacheTwoStepQuery.DFLT_PAGE_SIZE : qry.pageSize();
  +            long qryReqId = reqIdGen.incrementAndGet();
   
  -        r.tbls = new ArrayList<>(qry.mapQueries().size());
  +            QueryRun r = new QueryRun();
   
  -        String space = cctx.name();
  +            r.pageSize = qry.pageSize() <= 0 ? GridCacheTwoStepQuery.DFLT_PAGE_SIZE : qry.pageSize();
   
  -        r.conn = (JdbcConnection)h2.connectionForSpace(space);
  +            r.tbls = new ArrayList<>(qry.mapQueries().size());
   
  -        // TODO    Add topology version.
  -        ClusterGroup dataNodes = ctx.grid().cluster().forDataNodes(space);
  +            String space = cctx.name();
   
  -        if (cctx.isReplicated() || qry.explain()) {
  -            assert qry.explain() || dataNodes.node(ctx.localNodeId()) == null : "We must be on a client node.";
  +            r.conn = (JdbcConnection)h2.connectionForSpace(space);
   
  -            // Select random data node to run query on a replicated data or get EXPLAIN PLAN from a single node.
  -            dataNodes = dataNodes.forRandom();
  -        }
  +            AffinityTopologyVersion topVer = h2.readyTopologyVersion();
   
  -        final Collection<ClusterNode> nodes = dataNodes.nodes();
  +            List<String> extraSpaces = extraSpaces(space, qry.spaces());
   
  -        for (GridCacheSqlQuery mapQry : qry.mapQueries()) {
  -            GridMergeTable tbl;
  +            Collection<ClusterNode> nodes;
  +
  +            // Explicit partition mapping for unstable topology.
  +            Map<ClusterNode, IntArray> partsMap = null;
  +
  +            if (isPreloadingActive(cctx, extraSpaces)) {
  +                if (cctx.isReplicated())
 -                     nodes = replicatedDataNodes(cctx, extraSpaces);
 ++                    nodes = replicatedUnstableDataNodes(cctx, extraSpaces);
  +                else {
 -                     partsMap = partitionLocations(cctx, extraSpaces);
 ++                    partsMap = partitionedUnstableDataNodes(cctx, extraSpaces);
  +
  +                    nodes = partsMap == null ? null : partsMap.keySet();
  +                }
  +            }
  +            else
  +                nodes = stableDataNodes(topVer, cctx, extraSpaces);
  +
  +            if (nodes == null)
  +                continue; // Retry.
  +
  +            assert !nodes.isEmpty();
  +
  +            if (cctx.isReplicated() || qry.explain()) {
-                  assert qry.explain() || !nodes.contains(ctx.cluster().get().localNode()) : "We must be on a client node.";
+ +                assert qry.explain() || !nodes.contains(ctx.cluster().get().localNode()) :
+ +                    "We must be on a client node.";
  +
  +                // Select random data node to run query on a replicated data or get EXPLAIN PLAN from a single node.
  +                nodes = Collections.singleton(F.rand(nodes));
  +            }
  +
  +            for (GridCacheSqlQuery mapQry : qry.mapQueries()) {
  +                GridMergeTable tbl;
  +
  +                try {
  +                    tbl = createFunctionTable(r.conn, mapQry, qry.explain()); // createTable(r.conn, mapQry); TODO
  +                }
  +                catch (IgniteCheckedException e) {
  +                    throw new IgniteException(e);
  +                }
  +
  +                GridMergeIndex idx = tbl.getScanIndex(null);
  +
  +                for (ClusterNode node : nodes)
  +                    idx.addSource(node.id());
  +
  +                r.tbls.add(tbl);
  +
  +                curFunTbl.set(tbl);
  +            }
  +
  +            r.latch = new CountDownLatch(r.tbls.size() * nodes.size());
  +
  +            runs.put(qryReqId, r);
   
               try {
  -                tbl = createFunctionTable(r.conn, mapQry, qry.explain()); // createTable(r.conn, mapQry); TODO
  +                Collection<GridCacheSqlQuery> mapQrys = qry.mapQueries();
  +
  +                if (qry.explain()) {
  +                    mapQrys = new ArrayList<>(qry.mapQueries().size());
  +
  +                    for (GridCacheSqlQuery mapQry : qry.mapQueries())
  +                        mapQrys.add(new GridCacheSqlQuery(mapQry.alias(), "EXPLAIN " + mapQry.query(), mapQry.parameters()));
  +                }
  +
  +                if (nodes.size() != 1 || !F.first(nodes).isLocal()) { // Marshall params for remotes.
  +                    Marshaller m = ctx.config().getMarshaller();
  +
  +                    for (GridCacheSqlQuery mapQry : mapQrys)
  +                        mapQry.marshallParams(m);
  +                }
  +
  +                boolean retry = false;
  +
  +                if (send(nodes,
  +                    new GridQueryRequest(qryReqId, r.pageSize, space, mapQrys, topVer, extraSpaces, null), partsMap)) {
  +                    U.await(r.latch);
  +
  +                    Object state = r.state.get();
  +
  +                    if (state != null) {
  +                        if (state instanceof CacheException)
  +                            throw new CacheException("Failed to run map query remotely.", (CacheException)state);
  +
  +                        if (state instanceof AffinityTopologyVersion) {
  +                            retry = true;
  +
  +                            // If remote node asks us to retry then we have outdated full partition map.
  +                            h2.awaitForReadyTopologyVersion((AffinityTopologyVersion)state);
  +                        }
  +                    }
  +                }
  +                else // Send failed.
  +                    retry = true;
  +
  +                ResultSet res = null;
  +
  +                if (!retry) {
  +                    if (qry.explain())
  +                        return explainPlan(r.conn, space, qry);
  +
  +                    GridCacheSqlQuery rdc = qry.reduceQuery();
  +
  +                    res = h2.executeSqlQueryWithTimer(space, r.conn, rdc.query(), F.asList(rdc.parameters()));
  +                }
  +
  +                for (GridMergeTable tbl : r.tbls) {
  +                    if (!tbl.getScanIndex(null).fetchedAll()) // We have to explicitly cancel queries on remote nodes.
  +                        send(nodes, new GridQueryCancelRequest(qryReqId), null);
  +
  +//                dropTable(r.conn, tbl.getName()); TODO
  +                }
  +
  +                if (retry) {
  +                    if (Thread.currentThread().isInterrupted())
  +                        throw new IgniteInterruptedCheckedException("Query was interrupted.");
  +
  +                    continue;
  +                }
  +
  +                return new GridQueryCacheObjectsIterator(new Iter(res), cctx, keepPortable);
               }
  -            catch (IgniteCheckedException e) {
  -                throw new IgniteException(e);
  +            catch (IgniteCheckedException | RuntimeException e) {
  +                U.closeQuiet(r.conn);
  +
 -                 if (e instanceof CacheException)
 -                     throw (CacheException)e;
 - 
  +                throw new CacheException("Failed to run reduce query locally.", e);
  +            }
  +            finally {
  +                if (!runs.remove(qryReqId, r))
  +                    U.warn(log, "Query run was already removed: " + qryReqId);
  +
  +                curFunTbl.remove();
               }
  +        }
  +    }
  +
  +    /**
  +     * Calculates data nodes for replicated caches on unstable topology.
  +     *
  +     * @param cctx Cache context for main space.
  +     * @param extraSpaces Extra spaces.
  +     * @return Collection of all data nodes owning all the caches or {@code null} for retry.
  +     */
 -     private Collection<ClusterNode> replicatedDataNodes(final GridCacheContext<?,?> cctx, List<String> extraSpaces) {
 ++    private Collection<ClusterNode> replicatedUnstableDataNodes(final GridCacheContext<?,?> cctx,
 ++        List<String> extraSpaces) {
  +        assert cctx.isReplicated() : cctx.name() + " must be replicated";
  +
 -         Set<ClusterNode> nodes = owningReplicatedDataNodes(cctx);
 ++        Set<ClusterNode> nodes = replicatedUnstableDataNodes(cctx);
 + 
  -            GridMergeIndex idx = tbl.getScanIndex(null);
 ++        if (F.isEmpty(nodes))
 ++            return null; // Retry.
   
  -            for (ClusterNode node : nodes)
  -                idx.addSource(node.id());
  +        if (!F.isEmpty(extraSpaces)) {
  +            for (String extraSpace : extraSpaces) {
  +                GridCacheContext<?,?> extraCctx = cacheContext(extraSpace);
   
  -            r.tbls.add(tbl);
  +                if (extraCctx.isLocal())
  +                    continue;
   
  -            curFunTbl.set(tbl);
  +                if (!extraCctx.isReplicated())
  +                    throw new CacheException("Queries running on replicated cache should not contain JOINs " +
  +                        "with partitioned tables.");
  +
 -                 nodes.retainAll(owningReplicatedDataNodes(extraCctx));
 ++                Set<ClusterNode> extraOwners = replicatedUnstableDataNodes(extraCctx);
 ++
 ++                if (F.isEmpty(extraOwners))
 ++                    return null; // Retry.
 ++
 ++                nodes.retainAll(extraOwners);
  +
  +                if (nodes.isEmpty())
  +                    return null; // Retry.
  +            }
           }
   
  -        r.latch = new CountDownLatch(r.tbls.size() * nodes.size());
  +        return nodes;
  +    }
  +
  +    /**
 ++     * @param space Cache name.
 ++     * @param topVer Topology version.
 ++     * @return Collection of data nodes.
 ++     */
 ++    private Collection<ClusterNode> dataNodes(String space, AffinityTopologyVersion topVer) {
 ++        Collection<ClusterNode> res = ctx.discovery().cacheAffinityNodes(space, topVer);
 + 
  -        runs.put(qryReqId, r);
 ++        return res != null ? res : Collections.<ClusterNode>emptySet();
 ++    }
 + 
  -        try {
  -            Collection<GridCacheSqlQuery> mapQrys = qry.mapQueries();
 ++    /**
  +     * Collects all the nodes owning all the partitions for the given replicated cache.
  +     *
  +     * @param cctx Cache context.
 -      * @return Owning nodes.
 ++     * @return Owning nodes or {@code null} if we can't find owners for some partitions.
  +     */
 -     private Set<ClusterNode> owningReplicatedDataNodes(GridCacheContext<?,?> cctx) {
 ++    private Set<ClusterNode> replicatedUnstableDataNodes(GridCacheContext<?,?> cctx) {
  +        assert cctx.isReplicated() : cctx.name() + " must be replicated";
  +
  +        String space = cctx.name();
  +
 -         Set<ClusterNode> dataNodes = new HashSet<>(ctx.discovery().cacheAffinityNodes(space, NONE));
 ++        Set<ClusterNode> dataNodes = new HashSet<>(dataNodes(space, NONE));
  +
  +        if (dataNodes.isEmpty())
  +            throw new CacheException("No data nodes found for cache '" + space + "'");
  +
  +        // Find all the nodes owning all the partitions for replicated cache.
 -         for (int p = 0, extraParts = cctx.affinity().partitions(); p < extraParts; p++) {
 ++        for (int p = 0, parts = cctx.affinity().partitions(); p < parts; p++) {
  +            List<ClusterNode> owners = cctx.topology().owners(p);
  +
 -             if (owners.isEmpty())
 -                 throw new CacheException("No data nodes found for cache '" + space +
 -                     "' for partition " + p);
 ++            if (F.isEmpty(owners))
 ++                return null; // Retry.
  +
  +            dataNodes.retainAll(owners);
  +
  +            if (dataNodes.isEmpty())
 -                 throw new CacheException("No data nodes found for cache '" + space +
 -                     "' owning all the partitions.");
 ++                return null; // Retry.
  +        }
  +
  +        return dataNodes;
  +    }
  +
  +    /**
  +     * Calculates partition mapping for partitioned cache on unstable topology.
  +     *
  +     * @param cctx Cache context for main space.
  +     * @param extraSpaces Extra spaces.
  +     * @return Partition mapping or {@code null} if we can't calculate it due to repartitioning and we need to retry.
  +     */
  +    @SuppressWarnings("unchecked")
 -     private Map<ClusterNode, IntArray> partitionLocations(final GridCacheContext<?,?> cctx, List<String> extraSpaces) {
 ++    private Map<ClusterNode, IntArray> partitionedUnstableDataNodes(final GridCacheContext<?,?> cctx,
 ++        List<String> extraSpaces) {
  +        assert !cctx.isReplicated() && !cctx.isLocal() : cctx.name() + " must be partitioned";
  +
  +        final int partsCnt = cctx.affinity().partitions();
  +
  +        if (extraSpaces != null) { // Check correct number of partitions for partitioned caches.
  +            for (String extraSpace : extraSpaces) {
  +                GridCacheContext<?,?> extraCctx = cacheContext(extraSpace);
   
  -            if (qry.explain()) {
  -                mapQrys = new ArrayList<>(qry.mapQueries().size());
  +                if (extraCctx.isReplicated() || extraCctx.isLocal())
  +                    continue;
   
  -                for (GridCacheSqlQuery mapQry : qry.mapQueries())
  -                    mapQrys.add(new GridCacheSqlQuery(mapQry.alias(), "EXPLAIN " + mapQry.query(), mapQry.parameters()));
  +                int parts = extraCctx.affinity().partitions();
  +
  +                if (parts != partsCnt)
  +                    throw new CacheException("Number of partitions must be the same for correct collocation in " +
  +                        "caches " + cctx.name() + " and " + extraSpace + ".");
               }
  +        }
  +
  +        Set<ClusterNode>[] partLocs = new Set[partsCnt];
   
  -            if (nodes.size() != 1 || !F.first(nodes).isLocal()) { // Marshall params for remotes.
  -                Marshaller m = ctx.config().getMarshaller();
  +        // Fill partition locations for main cache.
  +        for (int p = 0, parts =  cctx.affinity().partitions(); p < parts; p++) {
  +            List<ClusterNode> owners = cctx.topology().owners(p);
   
 -             if (F.isEmpty(owners))
  -                for (GridCacheSqlQuery mapQry : mapQrys)
  -                    mapQry.marshallParams(m);
 ++            if (F.isEmpty(owners)) {
 ++                if (!F.isEmpty(dataNodes(cctx.name(), NONE)))
 ++                    return null; // Retry.
 ++
  +                throw new CacheException("No data nodes found for cache '" + cctx.name() + "' for partition " + p);
 +             }
   
  -            send(nodes, new GridQueryRequest(qryReqId, r.pageSize, space, mapQrys));
  +            partLocs[p] = new HashSet<>(owners);
  +        }
   
  -            r.latch.await();
  +        if (extraSpaces != null) {
  +            // Find owner intersections for each participating partitioned cache partition.
  +            // We need this for logical collocation between different partitioned caches with the same affinity.
  +            for (String extraSpace : extraSpaces) {
  +                GridCacheContext<?,?> extraCctx = cacheContext(extraSpace);
   
  -            if (r.rmtErr != null)
  -                throw new CacheException("Failed to run map query remotely.", r.rmtErr);
  +                if (extraCctx.isReplicated() || extraCctx.isLocal())
  +                    continue;
   
  -            if (qry.explain())
  -                return explainPlan(r.conn, space, qry);
  +                for (int p = 0, parts =  extraCctx.affinity().partitions(); p < parts; p++) {
  +                    List<ClusterNode> owners = extraCctx.topology().owners(p);
   
 -                     if (F.isEmpty(owners))
  -            GridCacheSqlQuery rdc = qry.reduceQuery();
 ++                    if (F.isEmpty(owners)) {
 ++                        if (!F.isEmpty(dataNodes(extraSpace, NONE)))
 ++                            return null; // Retry.
 + 
  -            final ResultSet res = h2.executeSqlQueryWithTimer(space, r.conn, rdc.query(), F.asList(rdc.parameters()));
  +                        throw new CacheException("No data nodes found for cache '" + extraSpace +
  +                            "' for partition " + p);
 ++                    }
   
  -            for (GridMergeTable tbl : r.tbls) {
  -                if (!tbl.getScanIndex(null).fetchedAll()) // We have to explicitly cancel queries on remote nodes.
  -                    send(nodes, new GridQueryCancelRequest(qryReqId));
  +                    if (partLocs[p] == null)
  +                        partLocs[p] = new HashSet<>(owners);
  +                    else {
  +                        partLocs[p].retainAll(owners); // Intersection of owners.
   
  -//                dropTable(r.conn, tbl.getName()); TODO
  +                        if (partLocs[p].isEmpty())
  +                            return null; // Intersection is empty -> retry.
  +                    }
  +                }
               }
   
  -            return new GridQueryCacheObjectsIterator(new Iter(res), cctx, keepPortable);
  +            // Filter nodes where not all the replicated caches loaded.
  +            for (String extraSpace : extraSpaces) {
  +                GridCacheContext<?,?> extraCctx = cacheContext(extraSpace);
  +
  +                if (!extraCctx.isReplicated())
  +                    continue;
  +
 -                 Set<ClusterNode> dataNodes = owningReplicatedDataNodes(extraCctx);
 ++                Set<ClusterNode> dataNodes = replicatedUnstableDataNodes(extraCctx);
 ++
 ++                if (F.isEmpty(dataNodes))
 ++                    return null; // Retry.
  +
  +                for (Set<ClusterNode> partLoc : partLocs) {
  +                    partLoc.retainAll(dataNodes);
  +
  +                    if (partLoc.isEmpty())
  +                        return null; // Retry.
  +                }
  +            }
           }
  -        catch (IgniteCheckedException | InterruptedException | RuntimeException e) {
  -            U.closeQuiet(r.conn);
   
  -            if (e instanceof CacheException)
  -                throw (CacheException)e;
  +        // Collect the final partitions mapping.
  +        Map<ClusterNode, IntArray> res = new HashMap<>();
  +
  +        // Here partitions in all IntArray's will be sorted in ascending order, this is important.
  +        for (int p = 0; p < partLocs.length; p++) {
  +            Set<ClusterNode> pl = partLocs[p];
   
  -            throw new CacheException("Failed to run reduce query locally.", e);
  +            assert !F.isEmpty(pl) : pl;
  +
  +            ClusterNode n = pl.size() == 1 ? F.first(pl) : F.rand(pl);
  +
  +            IntArray parts = res.get(n);
  +
  +            if (parts == null)
  +                res.put(n, parts = new IntArray());
  +
  +            parts.add(p);
           }
  -        finally {
  -            if (!runs.remove(qryReqId, r))
  -                U.warn(log, "Query run was already removed: " + qryReqId);
   
  -            curFunTbl.remove();
  +        return res;
  +    }
  +
  +    /**
  +     * @param mainSpace Main space.
  +     * @param allSpaces All spaces.
  +     * @return List of all extra spaces or {@code null} if none.
  +     */
  +    private List<String> extraSpaces(String mainSpace, Set<String> allSpaces) {
  +        if (F.isEmpty(allSpaces) || (allSpaces.size() == 1 && allSpaces.contains(mainSpace)))
  +            return null;
  +
  +        ArrayList<String> res = new ArrayList<>(allSpaces.size());
  +
  +        for (String space : allSpaces) {
  +            if (!F.eq(space, mainSpace))
  +                res.add(space);
           }
  +
  +        return res;
       }
   
       /**


[03/10] incubator-ignite git commit: # ignite-1020 properly skip non-primary entries for store update , tests

Posted by se...@apache.org.
# ignite-1020 properly skip non-primary entries for store update , tests


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

Branch: refs/heads/ignite-484-1
Commit: a75b868605f661a71ed96f17e9d36471ccbfcd13
Parents: 9a0e659
Author: sboikov <sb...@gridgain.com>
Authored: Wed Jun 17 11:25:23 2015 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Wed Jun 17 12:07:37 2015 +0300

----------------------------------------------------------------------
 .../transactions/IgniteTxLocalAdapter.java      |  12 +-
 .../CacheStoreUsageMultinodeAbstractTest.java   | 305 +++++++++++++++++++
 ...eUsageMultinodeDynamicStartAbstractTest.java | 169 ++++++++++
 ...oreUsageMultinodeDynamicStartAtomicTest.java |  32 ++
 ...heStoreUsageMultinodeDynamicStartTxTest.java |  32 ++
 ...reUsageMultinodeStaticStartAbstractTest.java | 158 ++++++++++
 ...toreUsageMultinodeStaticStartAtomicTest.java |  32 ++
 ...cheStoreUsageMultinodeStaticStartTxTest.java |  32 ++
 .../testsuites/IgniteCacheTestSuite4.java       |   4 +
 9 files changed, 772 insertions(+), 4 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/a75b8686/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/transactions/IgniteTxLocalAdapter.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/transactions/IgniteTxLocalAdapter.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/transactions/IgniteTxLocalAdapter.java
index bc6308b..7bc9042 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/transactions/IgniteTxLocalAdapter.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/transactions/IgniteTxLocalAdapter.java
@@ -500,12 +500,16 @@ public abstract class IgniteTxLocalAdapter extends IgniteTxAdapter
                     List<Object> rmvCol = null;
                     CacheStoreManager writeStore = null;
 
-                    boolean skipNear = near() && isWriteToStoreFromDht;
+                    boolean skipNonPrimary = near() && isWriteToStoreFromDht;
 
                     for (IgniteTxEntry e : writeEntries) {
-                        boolean skip = (skipNear && e.cached().isNear()) ||
-                            e.skipStore() ||
-                            (e.context().store().isLocal() && !e.context().affinityNode());
+                        boolean skip = e.skipStore();
+
+                        if (!skip && skipNonPrimary) {
+                            skip = e.cached().isNear() ||
+                                e.cached().detached() ||
+                                !e.context().affinity().primary(e.cached().partition(), topologyVersion()).isLocal();
+                        }
 
                         if (skip)
                             continue;

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/a75b8686/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheStoreUsageMultinodeAbstractTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheStoreUsageMultinodeAbstractTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheStoreUsageMultinodeAbstractTest.java
new file mode 100644
index 0000000..79a0293
--- /dev/null
+++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheStoreUsageMultinodeAbstractTest.java
@@ -0,0 +1,305 @@
+/*
+ * 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.cache.affinity.*;
+import org.apache.ignite.cache.store.*;
+import org.apache.ignite.cluster.*;
+import org.apache.ignite.configuration.*;
+import org.apache.ignite.internal.processors.cache.store.*;
+import org.apache.ignite.internal.util.lang.*;
+import org.apache.ignite.resources.*;
+import org.apache.ignite.spi.discovery.tcp.*;
+import org.apache.ignite.spi.discovery.tcp.ipfinder.*;
+import org.apache.ignite.spi.discovery.tcp.ipfinder.vm.*;
+import org.apache.ignite.testframework.*;
+import org.apache.ignite.testframework.junits.common.*;
+import org.apache.ignite.transactions.*;
+import org.jetbrains.annotations.*;
+
+import javax.cache.*;
+import javax.cache.configuration.*;
+import javax.cache.integration.*;
+import java.util.*;
+
+import static org.apache.ignite.cache.CacheAtomicityMode.*;
+import static org.apache.ignite.cache.CacheMode.*;
+import static org.apache.ignite.cache.CacheWriteSynchronizationMode.*;
+import static org.apache.ignite.internal.IgniteNodeAttributes.*;
+import static org.apache.ignite.transactions.TransactionIsolation.*;
+
+/**
+ *
+ */
+public abstract class CacheStoreUsageMultinodeAbstractTest extends GridCommonAbstractTest {
+    /** */
+    private static final TcpDiscoveryIpFinder IP_FINDER = new TcpDiscoveryVmIpFinder(true);
+
+    /** */
+    protected boolean client;
+
+    /** */
+    protected boolean cache;
+
+    /** */
+    protected boolean cacheStore;
+
+    /** */
+    protected boolean locStore;
+
+    /** */
+    protected boolean writeBehind;
+
+    /** */
+    protected boolean nearCache;
+
+    /** */
+    protected static Map<String, List<Cache.Entry<?, ?>>> writeMap;
+
+    /** {@inheritDoc} */
+    @Override protected IgniteConfiguration getConfiguration(String gridName) throws Exception {
+        IgniteConfiguration cfg = super.getConfiguration(gridName);
+
+        cfg.setClientMode(client);
+
+        ((TcpDiscoverySpi)cfg.getDiscoverySpi()).setIpFinder(IP_FINDER);
+
+        if (cache)
+            cfg.setCacheConfiguration(cacheConfiguration());
+
+        return cfg;
+    }
+
+    /**
+     * @return Cache configuration.
+     */
+    @SuppressWarnings("unchecked")
+    protected CacheConfiguration cacheConfiguration() {
+        CacheConfiguration ccfg = new CacheConfiguration();
+
+        ccfg.setCacheMode(PARTITIONED);
+        ccfg.setAtomicityMode(atomicityMode());
+        ccfg.setBackups(1);
+        ccfg.setWriteSynchronizationMode(FULL_SYNC);
+
+        if (cacheStore) {
+            if (writeBehind) {
+                ccfg.setWriteBehindEnabled(true);
+                ccfg.setWriteBehindFlushFrequency(100);
+            }
+
+            ccfg.setWriteThrough(true);
+
+            ccfg.setCacheStoreFactory(locStore ? new TestLocalStoreFactory() : new TestStoreFactory());
+        }
+
+        if (nearCache)
+            ccfg.setNearConfiguration(new NearCacheConfiguration());
+
+        return ccfg;
+    }
+
+    /**
+     * @return Cache atomicity mode.
+     */
+    protected abstract CacheAtomicityMode atomicityMode();
+
+    /** {@inheritDoc} */
+    @Override protected void beforeTest() throws Exception {
+        super.beforeTest();
+
+        writeMap = new HashMap<>();
+    }
+
+    /**
+     * @param clientStore {@code True} if store configured on client node.
+     * @throws Exception If failed.
+     */
+    protected void checkStoreUpdate(boolean clientStore) throws Exception {
+        Ignite client = grid(3);
+
+        assertTrue(client.configuration().isClientMode());
+
+        awaitPartitionMapExchange();
+
+        IgniteCache<Object, Object> cache0 = ignite(0).cache(null);
+        IgniteCache<Object, Object> cache1 = ignite(1).cache(null);
+        IgniteCache<Object, Object> clientCache = client.cache(null);
+
+        assertTrue(((IgniteCacheProxy)cache0).context().store().configured());
+        assertEquals(clientStore, ((IgniteCacheProxy) clientCache).context().store().configured());
+
+        List<TransactionConcurrency> tcList = new ArrayList<>();
+
+        tcList.add(null);
+
+        if (atomicityMode() == TRANSACTIONAL) {
+            tcList.add(TransactionConcurrency.OPTIMISTIC);
+            tcList.add(TransactionConcurrency.PESSIMISTIC);
+        }
+
+        log.info("Start test [atomicityMode=" + atomicityMode() +
+            ", locStore=" + locStore +
+            ", writeBehind=" + writeBehind +
+            ", nearCache=" + nearCache +
+            ", clientStore=" + clientStore + ']');
+
+        for (TransactionConcurrency tc : tcList) {
+            testStoreUpdate(cache0, primaryKey(cache0), tc);
+
+            testStoreUpdate(cache0, backupKey(cache0), tc);
+
+            testStoreUpdate(cache0, nearKey(cache0), tc);
+
+            testStoreUpdate(cache0, primaryKey(cache1), tc);
+
+            testStoreUpdate(clientCache, primaryKey(cache0), tc);
+
+            testStoreUpdate(clientCache, primaryKey(cache1), tc);
+        }
+    }
+
+    /**
+     * @param cache Cache.
+     * @param key Key.
+     * @param tc Transaction concurrency mode.
+     * @throws Exception If failed.
+     */
+    protected void testStoreUpdate(IgniteCache<Object, Object> cache,
+       Object key,
+       @Nullable TransactionConcurrency tc)
+        throws Exception
+    {
+        boolean storeOnPrimary = atomicityMode() == ATOMIC || locStore || writeBehind;
+
+        assertTrue(writeMap.isEmpty());
+
+        Ignite ignite = cache.unwrap(Ignite.class);
+
+        Affinity<Object> obj = ignite.affinity(cache.getName());
+
+        ClusterNode node = obj.mapKeyToNode(key);
+
+        assertNotNull(node);
+
+        String expNode = storeOnPrimary ? (String)node.attribute(ATTR_GRID_NAME) : ignite.name();
+
+        assertNotNull(expNode);
+
+        log.info("Put [node=" + ignite.name() +
+            ", key=" + key +
+            ", primary=" + node.attribute(ATTR_GRID_NAME) +
+            ", tx=" + tc +
+            ", nearCache=" + (cache.getConfiguration(CacheConfiguration.class).getNearConfiguration() != null) +
+            ", storeOnPrimary=" + storeOnPrimary + ']');
+
+        Transaction tx = tc != null ? ignite.transactions().txStart(tc, REPEATABLE_READ) : null;
+
+        cache.put(key, key);
+
+        if (tx != null)
+            tx.commit();
+
+        boolean wait = GridTestUtils.waitForCondition(new GridAbsPredicate() {
+            @Override
+            public boolean apply() {
+                return writeMap.size() > 0;
+            }
+        }, 1000);
+
+        assertTrue("Store is not updated", wait);
+
+        assertEquals("Write on wrong node: " + writeMap, 1, writeMap.size());
+
+        assertEquals(expNode, writeMap.keySet().iterator().next());
+
+        writeMap.clear();
+    }
+
+    /**
+     *
+     */
+    public static class TestStoreFactory implements Factory<CacheStore> {
+        /** {@inheritDoc} */
+        @Override public CacheStore create() {
+            return new TestStore();
+        }
+    }
+
+    /**
+     *
+     */
+    public static class TestLocalStoreFactory implements Factory<CacheStore> {
+        /** {@inheritDoc} */
+        @Override public CacheStore create() {
+            return new TestLocalStore();
+        }
+    }
+
+    /**
+     *
+     */
+    public static class TestStore extends CacheStoreAdapter<Object, Object> {
+        /** */
+        @IgniteInstanceResource
+        private Ignite ignite;
+
+        /** {@inheritDoc} */
+        @SuppressWarnings("SynchronizeOnNonFinalField")
+        @Override public void write(Cache.Entry<?, ?> entry) {
+            synchronized (writeMap) {
+                ignite.log().info("Write [node=" + ignite.name() + ", entry=" + entry + ']');
+
+                String name = ignite.name();
+
+                List<Cache.Entry<?, ?>> list = writeMap.get(name);
+
+                if (list == null) {
+                    list = new ArrayList<>();
+
+                    writeMap.put(name, list);
+                }
+
+                list.add(entry);
+            }
+        }
+
+        /** {@inheritDoc} */
+        @Override public Object load(Object key) throws CacheLoaderException {
+            throw new UnsupportedOperationException();
+        }
+
+        /** {@inheritDoc} */
+        @Override public void delete(Object key) {
+            throw new UnsupportedOperationException();
+        }
+    }
+
+    /**
+     *
+     */
+    @CacheLocalStore
+    public static class TestLocalStore extends TestStore {
+        /** {@inheritDoc} */
+        @Override public void delete(Object key) {
+            // No-op.
+        }
+    }
+}

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/a75b8686/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheStoreUsageMultinodeDynamicStartAbstractTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheStoreUsageMultinodeDynamicStartAbstractTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheStoreUsageMultinodeDynamicStartAbstractTest.java
new file mode 100644
index 0000000..82667d9
--- /dev/null
+++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheStoreUsageMultinodeDynamicStartAbstractTest.java
@@ -0,0 +1,169 @@
+/*
+ * 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.configuration.*;
+
+/**
+ *
+ */
+public abstract class CacheStoreUsageMultinodeDynamicStartAbstractTest extends CacheStoreUsageMultinodeAbstractTest {
+    /** {@inheritDoc} */
+    @Override protected void beforeTestsStarted() throws Exception {
+        cache = false;
+
+        startGridsMultiThreaded(3);
+
+        client = true;
+
+        startGrid(3);
+    }
+
+    /** {@inheritDoc} */
+    @Override protected void afterTestsStopped() throws Exception {
+        super.afterTestsStopped();
+
+        stopAllGrids();
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testDynamicStart() throws Exception {
+        checkStoreWithDynamicStart(false);
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testDynamicStartNearEnabled() throws Exception {
+        nearCache = true;
+
+        checkStoreWithDynamicStart(false);
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testDynamicFromClientStart() throws Exception {
+        checkStoreWithDynamicStart(true);
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testDynamicStartFromClientNearEnabled() throws Exception {
+        nearCache = true;
+
+        checkStoreWithDynamicStart(true);
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testDynamicStartLocalStore() throws Exception {
+        locStore = true;
+
+        checkStoreWithDynamicStart(false);
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testDynamicStartFromClientLocalStore() throws Exception {
+        locStore = true;
+
+        checkStoreWithDynamicStart(true);
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testDynamicStartLocalStoreNearEnabled() throws Exception {
+        locStore = true;
+
+        nearCache = true;
+
+        checkStoreWithDynamicStart(false);
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testDynamicStartWriteBehindStore() throws Exception {
+        writeBehind = true;
+
+        checkStoreWithDynamicStart(false);
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testDynamicStartFromClientWriteBehindStore() throws Exception {
+        writeBehind = true;
+
+        checkStoreWithDynamicStart(true);
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testDynamicStartWriteBehindStoreNearEnabled() throws Exception {
+        writeBehind = true;
+
+        nearCache = true;
+
+        checkStoreWithDynamicStart(false);
+    }
+
+    /**
+     * @param clientStart {@code True} if start cache from client node.
+     * @throws Exception If failed.
+     */
+    private void checkStoreWithDynamicStart(boolean clientStart) throws Exception {
+        cacheStore = true;
+
+        CacheConfiguration ccfg = cacheConfiguration();
+
+        assertNotNull(ccfg.getCacheStoreFactory());
+
+        Ignite srv = ignite(0);
+
+        Ignite client = ignite(3);
+
+        Ignite node = clientStart ? client : srv;
+
+        IgniteCache cache = nearCache ? node.createCache(ccfg, new NearCacheConfiguration()) : node.createCache(ccfg);
+
+        assertNotNull(cache);
+
+        try {
+            if (nearCache)
+                client.createNearCache(null, new NearCacheConfiguration<>());
+
+            checkStoreUpdate(true);
+        }
+        finally {
+            cache = srv.cache(null);
+
+            if (cache != null)
+                cache.close();
+        }
+    }
+}

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/a75b8686/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheStoreUsageMultinodeDynamicStartAtomicTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheStoreUsageMultinodeDynamicStartAtomicTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheStoreUsageMultinodeDynamicStartAtomicTest.java
new file mode 100644
index 0000000..884dedd
--- /dev/null
+++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheStoreUsageMultinodeDynamicStartAtomicTest.java
@@ -0,0 +1,32 @@
+/*
+ * 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.cache.*;
+
+import static org.apache.ignite.cache.CacheAtomicityMode.*;
+
+/**
+ *
+ */
+public class CacheStoreUsageMultinodeDynamicStartAtomicTest extends CacheStoreUsageMultinodeDynamicStartAbstractTest {
+    /** {@inheritDoc} */
+    @Override protected CacheAtomicityMode atomicityMode() {
+        return ATOMIC;
+    }
+}

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/a75b8686/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheStoreUsageMultinodeDynamicStartTxTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheStoreUsageMultinodeDynamicStartTxTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheStoreUsageMultinodeDynamicStartTxTest.java
new file mode 100644
index 0000000..825b450
--- /dev/null
+++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheStoreUsageMultinodeDynamicStartTxTest.java
@@ -0,0 +1,32 @@
+/*
+ * 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.cache.*;
+
+import static org.apache.ignite.cache.CacheAtomicityMode.*;
+
+/**
+ *
+ */
+public class CacheStoreUsageMultinodeDynamicStartTxTest extends CacheStoreUsageMultinodeDynamicStartAbstractTest {
+    /** {@inheritDoc} */
+    @Override protected CacheAtomicityMode atomicityMode() {
+        return TRANSACTIONAL;
+    }
+}

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/a75b8686/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheStoreUsageMultinodeStaticStartAbstractTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheStoreUsageMultinodeStaticStartAbstractTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheStoreUsageMultinodeStaticStartAbstractTest.java
new file mode 100644
index 0000000..c00d003
--- /dev/null
+++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheStoreUsageMultinodeStaticStartAbstractTest.java
@@ -0,0 +1,158 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License.  You may obtain a copy of the License at
+ *
+ *      http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.ignite.internal.processors.cache;
+
+import org.apache.ignite.*;
+import org.apache.ignite.transactions.*;
+
+import java.util.*;
+
+import static org.apache.ignite.cache.CacheAtomicityMode.*;
+
+/**
+ *
+ */
+public abstract class CacheStoreUsageMultinodeStaticStartAbstractTest extends CacheStoreUsageMultinodeAbstractTest {
+    /** {@inheritDoc} */
+    @Override protected void afterTest() throws Exception {
+        super.afterTest();
+
+        stopAllGrids();
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testStaticConfiguration() throws Exception {
+        checkStoreUpdateStaticConfig(true);
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testStaticConfigurationNearEnabled() throws Exception {
+        nearCache = true;
+
+        checkStoreUpdateStaticConfig(true);
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testStaticConfigurationLocalStore() throws Exception {
+        locStore = true;
+
+        checkStoreUpdateStaticConfig(true);
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testStaticConfigurationLocalStoreNearEnabled() throws Exception {
+        locStore = true;
+
+        nearCache = true;
+
+        checkStoreUpdateStaticConfig(true);
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testStaticConfigurationTxLocalStoreNoClientStore() throws Exception {
+        locStore = true;
+
+        checkStoreUpdateStaticConfig(false);
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testStaticConfigurationTxLocalStoreNoClientStoreNearEnabled() throws Exception {
+        locStore = true;
+
+        nearCache = true;
+
+        checkStoreUpdateStaticConfig(false);
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testStaticConfigurationTxWriteBehindStore() throws Exception {
+        writeBehind = true;
+
+        checkStoreUpdateStaticConfig(true);
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testStaticConfigurationTxWriteBehindStoreNearEnabled() throws Exception {
+        writeBehind = true;
+
+        nearCache = true;
+
+        checkStoreUpdateStaticConfig(true);
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testStaticConfigurationTxWriteBehindStoreNoClientStore() throws Exception {
+        writeBehind = true;
+
+        checkStoreUpdateStaticConfig(false);
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testStaticConfigurationTxWriteBehindStoreNoClientStoreNearEnabled() throws Exception {
+        writeBehind = true;
+
+        nearCache = true;
+
+        checkStoreUpdateStaticConfig(false);
+    }
+
+    /**
+     * @param clientStore {@code True} if store should be configured on client node.
+     * @throws Exception If failed.
+     */
+    private void checkStoreUpdateStaticConfig(boolean clientStore) throws Exception {
+        try {
+            cache = true;
+
+            cacheStore = true;
+
+            startGridsMultiThreaded(3);
+
+            client = true;
+
+            cacheStore = clientStore;
+
+            startGrid(3);
+
+            checkStoreUpdate(clientStore);
+        }
+        finally {
+            stopAllGrids();
+        }
+    }
+}

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/a75b8686/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheStoreUsageMultinodeStaticStartAtomicTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheStoreUsageMultinodeStaticStartAtomicTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheStoreUsageMultinodeStaticStartAtomicTest.java
new file mode 100644
index 0000000..6908aaa
--- /dev/null
+++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheStoreUsageMultinodeStaticStartAtomicTest.java
@@ -0,0 +1,32 @@
+/*
+ * 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.cache.*;
+
+import static org.apache.ignite.cache.CacheAtomicityMode.*;
+
+/**
+ *
+ */
+public class CacheStoreUsageMultinodeStaticStartAtomicTest extends CacheStoreUsageMultinodeStaticStartAbstractTest {
+    /** {@inheritDoc} */
+    @Override protected CacheAtomicityMode atomicityMode() {
+        return ATOMIC;
+    }
+}

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/a75b8686/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheStoreUsageMultinodeStaticStartTxTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheStoreUsageMultinodeStaticStartTxTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheStoreUsageMultinodeStaticStartTxTest.java
new file mode 100644
index 0000000..b42fd44
--- /dev/null
+++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheStoreUsageMultinodeStaticStartTxTest.java
@@ -0,0 +1,32 @@
+/*
+ * 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.cache.*;
+
+import static org.apache.ignite.cache.CacheAtomicityMode.*;
+
+/**
+ *
+ */
+public class CacheStoreUsageMultinodeStaticStartTxTest extends CacheStoreUsageMultinodeStaticStartAbstractTest {
+    /** {@inheritDoc} */
+    @Override protected CacheAtomicityMode atomicityMode() {
+        return TRANSACTIONAL;
+    }
+}

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/a75b8686/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteCacheTestSuite4.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteCacheTestSuite4.java b/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteCacheTestSuite4.java
index d155330..6295a4d 100644
--- a/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteCacheTestSuite4.java
+++ b/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteCacheTestSuite4.java
@@ -135,6 +135,10 @@ public class IgniteCacheTestSuite4 extends TestSuite {
         suite.addTestSuite(CacheJdbcStoreSessionListenerSelfTest.class);
 
         suite.addTestSuite(CacheClientStoreSelfTest.class);
+        suite.addTestSuite(CacheStoreUsageMultinodeStaticStartAtomicTest.class);
+        suite.addTestSuite(CacheStoreUsageMultinodeStaticStartTxTest.class);
+        suite.addTestSuite(CacheStoreUsageMultinodeDynamicStartAtomicTest.class);
+        suite.addTestSuite(CacheStoreUsageMultinodeDynamicStartTxTest.class);
 
         suite.addTestSuite(IgniteStartCacheInTransactionSelfTest.class);
         suite.addTestSuite(IgniteStartCacheInTransactionAtomicSelfTest.class);


[10/10] incubator-ignite git commit: ignite-484-1 - relax test: 2 restarting nodes instead of 3

Posted by se...@apache.org.
ignite-484-1 - relax test: 2 restarting nodes instead of 3


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

Branch: refs/heads/ignite-484-1
Commit: ed43dfe1c7e9b7db3f6ef33a38dad9b28c41190c
Parents: e602ca0
Author: S.Vladykin <sv...@gridgain.com>
Authored: Wed Jun 17 20:33:54 2015 +0300
Committer: S.Vladykin <sv...@gridgain.com>
Committed: Wed Jun 17 20:33:54 2015 +0300

----------------------------------------------------------------------
 .../near/IgniteCacheClientQueryReplicatedNodeRestartSelfTest.java  | 2 +-
 1 file changed, 1 insertion(+), 1 deletion(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/ed43dfe1/modules/indexing/src/test/java/org/apache/ignite/internal/processors/cache/distributed/near/IgniteCacheClientQueryReplicatedNodeRestartSelfTest.java
----------------------------------------------------------------------
diff --git a/modules/indexing/src/test/java/org/apache/ignite/internal/processors/cache/distributed/near/IgniteCacheClientQueryReplicatedNodeRestartSelfTest.java b/modules/indexing/src/test/java/org/apache/ignite/internal/processors/cache/distributed/near/IgniteCacheClientQueryReplicatedNodeRestartSelfTest.java
index 3f23005..b868cfb 100644
--- a/modules/indexing/src/test/java/org/apache/ignite/internal/processors/cache/distributed/near/IgniteCacheClientQueryReplicatedNodeRestartSelfTest.java
+++ b/modules/indexing/src/test/java/org/apache/ignite/internal/processors/cache/distributed/near/IgniteCacheClientQueryReplicatedNodeRestartSelfTest.java
@@ -194,7 +194,7 @@ public class IgniteCacheClientQueryReplicatedNodeRestartSelfTest extends GridCom
     public void testRestarts() throws Exception {
         int duration = 90 * 1000;
         int qryThreadNum = 5;
-        int restartThreadsNum = 3; // 3 of 4 data nodes
+        int restartThreadsNum = 2; // 2 of 4 data nodes
         final int nodeLifeTime = 2 * 1000;
         final int logFreq = 10;
 


[02/10] incubator-ignite git commit: Merge remote-tracking branch 'remotes/origin/ignite-sprint-6' into ignite-1020

Posted by se...@apache.org.
Merge remote-tracking branch 'remotes/origin/ignite-sprint-6' into ignite-1020


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

Branch: refs/heads/ignite-484-1
Commit: 9a0e659a847ab3ec7c03443e2ada28aac51a26ea
Parents: 7fd1441 d1a2150
Author: sboikov <sb...@gridgain.com>
Authored: Wed Jun 17 10:25:34 2015 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Wed Jun 17 10:25:34 2015 +0300

----------------------------------------------------------------------
 RELEASE_NOTES.txt                               |   2 +-
 .../cache/store/jdbc/CacheJdbcBlobStore.java    |  22 +-
 .../store/jdbc/CacheJdbcBlobStoreFactory.java   | 290 +++++++++++++++++++
 .../cache/store/jdbc/CacheJdbcPojoStore.java    |   6 +-
 .../store/jdbc/CacheJdbcPojoStoreFactory.java   | 148 ++++++++++
 .../configuration/CacheConfiguration.java       |   3 +-
 .../internal/interop/InteropBootstrap.java      |   3 +-
 .../internal/interop/InteropIgnition.java       |   5 +-
 .../processors/cache/GridCacheProcessor.java    |  23 +-
 .../util/spring/IgniteSpringHelper.java         |  10 +
 .../SpringApplicationContextResource.java       |   4 +-
 .../apache/ignite/resources/SpringResource.java |   6 +-
 modules/hibernate/pom.xml                       |  14 +
 .../hibernate/CacheHibernateBlobStore.java      |  87 +-----
 .../CacheHibernateBlobStoreFactory.java         | 235 +++++++++++++++
 .../hibernate/src/test/config/factory-cache.xml |  59 ++++
 .../src/test/config/factory-cache1.xml          |  61 ++++
 .../config/factory-incorrect-store-cache.xml    |  56 ++++
 .../CacheHibernateStoreFactorySelfTest.java     | 273 +++++++++++++++++
 .../testsuites/IgniteHibernateTestSuite.java    |   2 +
 modules/spring/pom.xml                          |   7 +
 .../GridResourceSpringBeanInjector.java         |   2 +-
 .../util/spring/IgniteSpringHelperImpl.java     |  17 ++
 .../src/test/config/incorrect-store-cache.xml   |  57 ++++
 modules/spring/src/test/config/node.xml         |  43 +++
 modules/spring/src/test/config/node1.xml        |  45 +++
 .../test/config/pojo-incorrect-store-cache.xml  |  56 ++++
 modules/spring/src/test/config/store-cache.xml  |  59 ++++
 modules/spring/src/test/config/store-cache1.xml |  62 ++++
 .../jdbc/CacheJdbcBlobStoreFactorySelfTest.java | 172 +++++++++++
 .../jdbc/CacheJdbcPojoStoreFactorySelfTest.java | 193 ++++++++++++
 .../testsuites/IgniteSpringTestSuite.java       |   5 +
 32 files changed, 1907 insertions(+), 120 deletions(-)
----------------------------------------------------------------------



[06/10] incubator-ignite git commit: Merge remote-tracking branch 'remotes/origin/ignite-1020' into ignite-sprint-6

Posted by se...@apache.org.
Merge remote-tracking branch 'remotes/origin/ignite-1020' into ignite-sprint-6


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

Branch: refs/heads/ignite-484-1
Commit: af829d0ee0a646ac7ed96c65ddda7fc6f38212b0
Parents: 5218210 674072a
Author: sboikov <sb...@gridgain.com>
Authored: Wed Jun 17 17:19:58 2015 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Wed Jun 17 17:19:58 2015 +0300

----------------------------------------------------------------------
 .../processors/cache/GridCacheAdapter.java      |   4 +
 .../processors/cache/IgniteCacheProxy.java      |   7 +
 .../dht/GridDhtTransactionalCacheAdapter.java   |   2 +-
 .../cache/transactions/IgniteTxHandler.java     |   2 +-
 .../transactions/IgniteTxLocalAdapter.java      |  12 +-
 .../dr/IgniteDrDataStreamerCacheUpdater.java    |   7 +-
 .../CacheStoreUsageMultinodeAbstractTest.java   | 305 +++++++++++++++++++
 ...eUsageMultinodeDynamicStartAbstractTest.java | 169 ++++++++++
 ...oreUsageMultinodeDynamicStartAtomicTest.java |  32 ++
 ...heStoreUsageMultinodeDynamicStartTxTest.java |  32 ++
 ...reUsageMultinodeStaticStartAbstractTest.java | 158 ++++++++++
 ...toreUsageMultinodeStaticStartAtomicTest.java |  32 ++
 ...cheStoreUsageMultinodeStaticStartTxTest.java |  32 ++
 .../testsuites/IgniteCacheTestSuite4.java       |   4 +
 14 files changed, 791 insertions(+), 7 deletions(-)
----------------------------------------------------------------------