You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@ignite.apache.org by ag...@apache.org on 2015/07/02 23:46:36 UTC

[01/13] incubator-ignite git commit: IGNITE-621 - Added automatic retries.

Repository: incubator-ignite
Updated Branches:
  refs/heads/ignite-sprint-7 a0a31e221 -> ea90d8633


IGNITE-621 - Added automatic retries.


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

Branch: refs/heads/ignite-sprint-7
Commit: 3787a9d3353c0c146141a79e3e87e1bbc5128031
Parents: 415264e
Author: Alexey Goncharuk <ag...@gridgain.com>
Authored: Fri Jun 19 17:15:02 2015 -0700
Committer: Alexey Goncharuk <ag...@gridgain.com>
Committed: Fri Jun 19 17:15:02 2015 -0700

----------------------------------------------------------------------
 .../java/org/apache/ignite/IgniteCache.java     |   5 +
 .../processors/cache/CacheOperationContext.java |  44 +++++-
 .../processors/cache/GridCacheAdapter.java      |  91 +++++++------
 .../processors/cache/GridCacheProxyImpl.java    |  10 +-
 .../processors/cache/IgniteCacheProxy.java      |  36 ++++-
 .../dht/atomic/GridDhtAtomicCache.java          |  18 ++-
 .../dht/atomic/GridNearAtomicUpdateFuture.java  |  87 ++++++++++--
 .../IgniteCachePutRetryAbstractSelfTest.java    | 134 +++++++++++++++++++
 .../dht/IgniteCachePutRetryAtomicSelfTest.java  |  34 +++++
 ...gniteCachePutRetryTransactionalSelfTest.java |  35 +++++
 10 files changed, 422 insertions(+), 72 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/3787a9d3/modules/core/src/main/java/org/apache/ignite/IgniteCache.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/IgniteCache.java b/modules/core/src/main/java/org/apache/ignite/IgniteCache.java
index 2b97e55..c8d6d7a 100644
--- a/modules/core/src/main/java/org/apache/ignite/IgniteCache.java
+++ b/modules/core/src/main/java/org/apache/ignite/IgniteCache.java
@@ -106,6 +106,11 @@ public interface IgniteCache<K, V> extends javax.cache.Cache<K, V>, IgniteAsyncS
     public IgniteCache<K, V> withSkipStore();
 
     /**
+     * @return Cache with no-retries behavior enabled.
+     */
+    public IgniteCache<K, V> withNoRetries();
+
+    /**
      * Executes {@link #localLoadCache(IgniteBiPredicate, Object...)} on all cache nodes.
      *
      * @param p Optional predicate (may be {@code null}). If provided, will be used to

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/3787a9d3/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/CacheOperationContext.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/CacheOperationContext.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/CacheOperationContext.java
index 34d2bf4..343a2f0 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/CacheOperationContext.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/CacheOperationContext.java
@@ -36,6 +36,10 @@ public class CacheOperationContext implements Serializable {
     @GridToStringInclude
     private final boolean skipStore;
 
+    /** No retries flag. */
+    @GridToStringInclude
+    private final boolean noRetries;
+
     /** Client ID which operates over this projection. */
     private final UUID subjId;
 
@@ -56,6 +60,8 @@ public class CacheOperationContext implements Serializable {
         keepPortable = false;
 
         expiryPlc = null;
+
+        noRetries = false;
     }
 
     /**
@@ -68,7 +74,8 @@ public class CacheOperationContext implements Serializable {
         boolean skipStore,
         @Nullable UUID subjId,
         boolean keepPortable,
-        @Nullable ExpiryPolicy expiryPlc) {
+        @Nullable ExpiryPolicy expiryPlc,
+        boolean noRetries) {
         this.skipStore = skipStore;
 
         this.subjId = subjId;
@@ -76,6 +83,8 @@ public class CacheOperationContext implements Serializable {
         this.keepPortable = keepPortable;
 
         this.expiryPlc = expiryPlc;
+
+        this.noRetries = noRetries;
     }
 
     /**
@@ -95,7 +104,8 @@ public class CacheOperationContext implements Serializable {
             skipStore,
             subjId,
             true,
-            expiryPlc);
+            expiryPlc,
+            noRetries);
     }
 
     /**
@@ -118,7 +128,8 @@ public class CacheOperationContext implements Serializable {
             skipStore,
             subjId,
             keepPortable,
-            expiryPlc);
+            expiryPlc,
+            noRetries);
     }
 
     /**
@@ -139,7 +150,8 @@ public class CacheOperationContext implements Serializable {
             skipStore,
             subjId,
             keepPortable,
-            expiryPlc);
+            expiryPlc,
+            noRetries);
     }
 
     /**
@@ -160,7 +172,29 @@ public class CacheOperationContext implements Serializable {
             skipStore,
             subjId,
             true,
-            plc);
+            plc,
+            noRetries);
+    }
+
+    /**
+     * @param noRetries No retries flag.
+     * @return Operation context.
+     */
+    public CacheOperationContext setNoRetries(boolean noRetries) {
+        return new CacheOperationContext(
+            skipStore,
+            subjId,
+            keepPortable,
+            expiryPlc,
+            noRetries
+        );
+    }
+
+    /**
+     * @return No retries flag.
+     */
+    public boolean noRetries() {
+        return noRetries;
     }
 
     /** {@inheritDoc} */

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/3787a9d3/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 7335d72..f993527 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
@@ -78,6 +78,9 @@ public abstract class GridCacheAdapter<K, V> implements IgniteInternalCache<K, V
     /** clearLocally() split threshold. */
     public static final int CLEAR_ALL_SPLIT_THRESHOLD = 10000;
 
+    /** Maximum number of retries when topology changes. */
+    public static final int MAX_RETRIES = 100;
+
     /** Deserialization stash. */
     private static final ThreadLocal<IgniteBiTuple<String, String>> stash = new ThreadLocal<IgniteBiTuple<String,
         String>>() {
@@ -363,7 +366,7 @@ public abstract class GridCacheAdapter<K, V> implements IgniteInternalCache<K, V
 
     /** {@inheritDoc} */
     @Override public GridCacheProxyImpl<K, V> forSubjectId(UUID subjId) {
-        CacheOperationContext opCtx = new CacheOperationContext(false, subjId, false, null);
+        CacheOperationContext opCtx = new CacheOperationContext(false, subjId, false, null, false);
 
         return new GridCacheProxyImpl<>(ctx, this, opCtx);
     }
@@ -375,14 +378,14 @@ public abstract class GridCacheAdapter<K, V> implements IgniteInternalCache<K, V
 
     /** {@inheritDoc} */
     @Override public GridCacheProxyImpl<K, V> setSkipStore(boolean skipStore) {
-        CacheOperationContext opCtx = new CacheOperationContext(true, null, false, null);
+        CacheOperationContext opCtx = new CacheOperationContext(true, null, false, null, false);
 
         return new GridCacheProxyImpl<>(ctx, this, opCtx);
     }
 
     /** {@inheritDoc} */
     @Override public <K1, V1> GridCacheProxyImpl<K1, V1> keepPortable() {
-        CacheOperationContext opCtx = new CacheOperationContext(false, null, true, null);
+        CacheOperationContext opCtx = new CacheOperationContext(false, null, true, null, false);
 
         return new GridCacheProxyImpl<>((GridCacheContext<K1, V1>)ctx, (GridCacheAdapter<K1, V1>)this, opCtx);
     }
@@ -399,7 +402,7 @@ public abstract class GridCacheAdapter<K, V> implements IgniteInternalCache<K, V
         assert !CU.isAtomicsCache(ctx.name());
         assert !CU.isMarshallerCache(ctx.name());
 
-        CacheOperationContext opCtx = new CacheOperationContext(false, null, false, plc);
+        CacheOperationContext opCtx = new CacheOperationContext(false, null, false, plc, false);
 
         return new GridCacheProxyImpl<>(ctx, this, opCtx);
     }
@@ -2301,7 +2304,7 @@ public abstract class GridCacheAdapter<K, V> implements IgniteInternalCache<K, V
      * @return Putx operation future.
      */
     public IgniteInternalFuture<Boolean> putAsync0(final K key, final V val,
-                                                   @Nullable final CacheEntryPredicate... filter) {
+        @Nullable final CacheEntryPredicate... filter) {
         A.notNull(key, "key", val, "val");
 
         if (keyCheck)
@@ -3930,51 +3933,63 @@ public abstract class GridCacheAdapter<K, V> implements IgniteInternalCache<K, V
         if (tx == null || tx.implicit()) {
             TransactionConfiguration tCfg = ctx.gridConfig().getTransactionConfiguration();
 
-            tx = ctx.tm().newTx(
-                true,
-                op.single(),
-                ctx.systemTx() ? ctx : null,
-                OPTIMISTIC,
-                READ_COMMITTED,
-                tCfg.getDefaultTxTimeout(),
-                !ctx.skipStore(),
-                0
-            );
+            CacheOperationContext opCtx = ctx.operationContextPerCall();
 
-            assert tx != null;
+            int retries = opCtx != null && opCtx.noRetries() ? 1 : MAX_RETRIES;
 
-            try {
-                T t = op.op(tx);
+            for (int i = 0; i < retries; i++) {
+                tx = ctx.tm().newTx(
+                    true,
+                    op.single(),
+                    ctx.systemTx() ? ctx : null,
+                    OPTIMISTIC,
+                    READ_COMMITTED,
+                    tCfg.getDefaultTxTimeout(),
+                    !ctx.skipStore(),
+                    0
+                );
 
-                assert tx.done() : "Transaction is not done: " + tx;
+                assert tx != null;
 
-                return t;
-            }
-            catch (IgniteInterruptedCheckedException | IgniteTxHeuristicCheckedException |
-                IgniteTxRollbackCheckedException e) {
-                throw e;
-            }
-            catch (IgniteCheckedException e) {
                 try {
-                    tx.rollback();
+                    T t = op.op(tx);
 
-                    e = new IgniteTxRollbackCheckedException("Transaction has been rolled back: " +
-                        tx.xid(), e);
+                    assert tx.done() : "Transaction is not done: " + tx;
+
+                    return t;
+                }
+                catch (IgniteInterruptedCheckedException | IgniteTxHeuristicCheckedException |
+                    IgniteTxRollbackCheckedException e) {
+                    throw e;
                 }
-                catch (IgniteCheckedException | AssertionError | RuntimeException e1) {
-                    U.error(log, "Failed to rollback transaction (cache may contain stale locks): " + tx, e1);
+                catch (IgniteCheckedException e) {
+                    try {
+                        tx.rollback();
+
+                        e = new IgniteTxRollbackCheckedException("Transaction has been rolled back: " +
+                            tx.xid(), e);
+                    }
+                    catch (IgniteCheckedException | AssertionError | RuntimeException e1) {
+                        U.error(log, "Failed to rollback transaction (cache may contain stale locks): " + tx, e1);
+
+                        U.addLastCause(e, e1, log);
+                    }
+
+                    if (X.hasCause(e, ClusterTopologyCheckedException.class) && i != retries - 1)
+                        continue;
 
-                    U.addLastCause(e, e1, log);
+                    throw e;
                 }
+                finally {
+                    ctx.tm().resetContext();
 
-                throw e;
+                    if (ctx.isNear())
+                        ctx.near().dht().context().tm().resetContext();
+                }
             }
-            finally {
-                ctx.tm().resetContext();
 
-                if (ctx.isNear())
-                    ctx.near().dht().context().tm().resetContext();
-            }
+            // Should not happen.
+            throw new IgniteCheckedException("Failed to perform cache operation (maximum number of retries exceeded).");
         }
         else
             return op.op(tx);

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/3787a9d3/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheProxyImpl.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheProxyImpl.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheProxyImpl.java
index 63ba242..cec8c53 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheProxyImpl.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheProxyImpl.java
@@ -118,7 +118,7 @@ public class GridCacheProxyImpl<K, V> implements IgniteInternalCache<K, V>, Exte
         CacheOperationContext prev = gate.enter(opCtx);
 
         try {
-            return opCtx != null ? opCtx.skipStore() : false;
+            return opCtx != null && opCtx.skipStore();
         }
         finally {
             gate.leave(prev);
@@ -198,7 +198,7 @@ public class GridCacheProxyImpl<K, V> implements IgniteInternalCache<K, V>, Exte
     /** {@inheritDoc} */
     @Override public GridCacheProxyImpl<K, V> forSubjectId(UUID subjId) {
         return new GridCacheProxyImpl<>(ctx, delegate,
-            opCtx != null ? opCtx.forSubjectId(subjId) : new CacheOperationContext(false, subjId, false, null));
+            opCtx != null ? opCtx.forSubjectId(subjId) : new CacheOperationContext(false, subjId, false, null, false));
     }
 
     /** {@inheritDoc} */
@@ -210,7 +210,7 @@ public class GridCacheProxyImpl<K, V> implements IgniteInternalCache<K, V>, Exte
                 return this;
 
             return new GridCacheProxyImpl<>(ctx, delegate,
-                opCtx != null ? opCtx.setSkipStore(skipStore) : new CacheOperationContext(true, null, false, null));
+                opCtx != null ? opCtx.setSkipStore(skipStore) : new CacheOperationContext(true, null, false, null, false));
         }
         finally {
             gate.leave(prev);
@@ -224,7 +224,7 @@ public class GridCacheProxyImpl<K, V> implements IgniteInternalCache<K, V>, Exte
         
         return new GridCacheProxyImpl<>((GridCacheContext<K1, V1>)ctx, 
             (GridCacheAdapter<K1, V1>)delegate,
-            opCtx != null ? opCtx.keepPortable() : new CacheOperationContext(false, null, true, null));
+            opCtx != null ? opCtx.keepPortable() : new CacheOperationContext(false, null, true, null, false));
     }
 
     /** {@inheritDoc} */
@@ -1515,7 +1515,7 @@ public class GridCacheProxyImpl<K, V> implements IgniteInternalCache<K, V>, Exte
 
         try {
             return new GridCacheProxyImpl<>(ctx, delegate,
-                opCtx != null ? opCtx.withExpiryPolicy(plc) : new CacheOperationContext(false, null, false, plc));
+                opCtx != null ? opCtx.withExpiryPolicy(plc) : new CacheOperationContext(false, null, false, plc, false));
         }
         finally {
             gate.leave(prev);

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/3787a9d3/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 48fd259..0ad2a9a 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
@@ -246,7 +246,7 @@ public class IgniteCacheProxy<K, V> extends AsyncSupportAdapter<IgniteCache<K, V
 
         try {
             CacheOperationContext prj0 = opCtx != null ? opCtx.withExpiryPolicy(plc) :
-                new CacheOperationContext(false, null, false, plc);
+                new CacheOperationContext(false, null, false, plc, false);
 
             return new IgniteCacheProxy<>(ctx, delegate, prj0, isAsync(), lock);
         }
@@ -261,6 +261,30 @@ public class IgniteCacheProxy<K, V> extends AsyncSupportAdapter<IgniteCache<K, V
     }
 
     /** {@inheritDoc} */
+    @Override public IgniteCache<K, V> withNoRetries() {
+        CacheOperationContext prev = onEnter(opCtx);
+
+        try {
+            boolean noRetries = opCtx != null && opCtx.noRetries();
+
+            if (noRetries)
+                return this;
+
+            CacheOperationContext opCtx0 = opCtx != null ? opCtx.setNoRetries(true) :
+                new CacheOperationContext(false, null, false, null, true);
+
+            return new IgniteCacheProxy<>(ctx,
+                delegate,
+                opCtx0,
+                isAsync(),
+                lock);
+        }
+        finally {
+            onLeave(prev);
+        }
+    }
+
+    /** {@inheritDoc} */
     @Override public void loadCache(@Nullable IgniteBiPredicate<K, V> p, @Nullable Object... args) {
         try {
             CacheOperationContext prev = onEnter(opCtx);
@@ -1498,10 +1522,11 @@ public class IgniteCacheProxy<K, V> extends AsyncSupportAdapter<IgniteCache<K, V
         try {
             CacheOperationContext opCtx0 =
                 new CacheOperationContext(
-                    opCtx != null ? opCtx.skipStore() : false,
+                    opCtx != null && opCtx.skipStore(),
                     opCtx != null ? opCtx.subjectId() : null,
                     true,
-                    opCtx != null ? opCtx.expiry() : null);
+                    opCtx != null ? opCtx.expiry() : null,
+                    opCtx != null && opCtx.noRetries());
 
             return new IgniteCacheProxy<>((GridCacheContext<K1, V1>)ctx,
                 (GridCacheAdapter<K1, V1>)delegate,
@@ -1529,8 +1554,9 @@ public class IgniteCacheProxy<K, V> extends AsyncSupportAdapter<IgniteCache<K, V
             CacheOperationContext opCtx0 =
                 new CacheOperationContext(true,
                     opCtx != null ? opCtx.subjectId() : null,
-                    opCtx != null ? opCtx.isKeepPortable() : false,
-                    opCtx != null ? opCtx.expiry() : null);
+                    opCtx != null && opCtx.isKeepPortable(),
+                    opCtx != null ? opCtx.expiry() : null,
+                    opCtx != null && opCtx.noRetries());
 
             return new IgniteCacheProxy<>(ctx,
                 delegate,

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/3787a9d3/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridDhtAtomicCache.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridDhtAtomicCache.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridDhtAtomicCache.java
index 8630421..2863ae8 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridDhtAtomicCache.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridDhtAtomicCache.java
@@ -767,11 +767,13 @@ public class GridDhtAtomicCache<K, V> extends GridDhtCacheAdapter<K, V> {
             filter,
             subjId,
             taskNameHash,
-            opCtx != null && opCtx.skipStore());
+            opCtx != null && opCtx.skipStore(),
+            opCtx != null && opCtx.noRetries() ? 1 : MAX_RETRIES,
+            waitTopFut);
 
         return asyncOp(new CO<IgniteInternalFuture<Object>>() {
             @Override public IgniteInternalFuture<Object> apply() {
-                updateFut.map(waitTopFut);
+                updateFut.map();
 
                 return updateFut;
             }
@@ -830,14 +832,16 @@ public class GridDhtAtomicCache<K, V> extends GridDhtCacheAdapter<K, V> {
             filter,
             subjId,
             taskNameHash,
-            opCtx != null && opCtx.skipStore());
+            opCtx != null && opCtx.skipStore(),
+            opCtx != null && opCtx.noRetries() ? 1 : MAX_RETRIES,
+            true);
 
         if (statsEnabled)
             updateFut.listen(new UpdateRemoveTimeStatClosure<>(metrics0(), start));
 
         return asyncOp(new CO<IgniteInternalFuture<Object>>() {
             @Override public IgniteInternalFuture<Object> apply() {
-                updateFut.map(true);
+                updateFut.map();
 
                 return updateFut;
             }
@@ -2273,9 +2277,11 @@ public class GridDhtAtomicCache<K, V> extends GridDhtCacheAdapter<K, V> {
             req.filter(),
             req.subjectId(),
             req.taskNameHash(),
-            req.skipStore());
+            req.skipStore(),
+            MAX_RETRIES,
+            true);
 
-        updateFut.map(true);
+        updateFut.map();
     }
 
     /**

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/3787a9d3/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridNearAtomicUpdateFuture.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridNearAtomicUpdateFuture.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridNearAtomicUpdateFuture.java
index 07f5ecf..53150cc 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridNearAtomicUpdateFuture.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridNearAtomicUpdateFuture.java
@@ -90,7 +90,7 @@ public class GridNearAtomicUpdateFuture extends GridFutureAdapter<Object>
 
     /** Mappings. */
     @GridToStringInclude
-    private final ConcurrentMap<UUID, GridNearAtomicUpdateRequest> mappings;
+    private ConcurrentMap<UUID, GridNearAtomicUpdateRequest> mappings;
 
     /** Error. */
     private volatile CachePartialUpdateCheckedException err;
@@ -123,7 +123,7 @@ public class GridNearAtomicUpdateFuture extends GridFutureAdapter<Object>
     private GridNearAtomicUpdateRequest singleReq;
 
     /** Raw return value flag. */
-    private boolean rawRetval;
+    private final boolean rawRetval;
 
     /** Fast map flag. */
     private final boolean fastMap;
@@ -149,6 +149,12 @@ public class GridNearAtomicUpdateFuture extends GridFutureAdapter<Object>
     /** Skip store flag. */
     private final boolean skipStore;
 
+    /** Wait for topology future flag. */
+    private final boolean waitTopFut;
+
+    /** Remap count. */
+    private AtomicInteger remapCnt;
+
     /**
      * @param cctx Cache context.
      * @param cache Cache instance.
@@ -183,7 +189,9 @@ public class GridNearAtomicUpdateFuture extends GridFutureAdapter<Object>
         final CacheEntryPredicate[] filter,
         UUID subjId,
         int taskNameHash,
-        boolean skipStore
+        boolean skipStore,
+        int remapCnt,
+        boolean waitTopFut
     ) {
         this.rawRetval = rawRetval;
 
@@ -207,6 +215,7 @@ public class GridNearAtomicUpdateFuture extends GridFutureAdapter<Object>
         this.subjId = subjId;
         this.taskNameHash = taskNameHash;
         this.skipStore = skipStore;
+        this.waitTopFut = waitTopFut;
 
         if (log == null)
             log = U.logger(cctx.kernalContext(), logRef, GridFutureAdapter.class);
@@ -218,6 +227,8 @@ public class GridNearAtomicUpdateFuture extends GridFutureAdapter<Object>
             !(cctx.writeThrough() && cctx.config().getInterceptor() != null);
 
         nearEnabled = CU.isNearEnabled(cctx);
+
+        this.remapCnt = new AtomicInteger(remapCnt);
     }
 
     /** {@inheritDoc} */
@@ -295,10 +306,8 @@ public class GridNearAtomicUpdateFuture extends GridFutureAdapter<Object>
 
     /**
      * Performs future mapping.
-     *
-     * @param waitTopFut Whether to wait for topology future.
      */
-    public void map(boolean waitTopFut) {
+    public void map() {
         AffinityTopologyVersion topVer = null;
 
         IgniteInternalTx tx = cctx.tm().anyActiveThreadTx();
@@ -310,14 +319,62 @@ public class GridNearAtomicUpdateFuture extends GridFutureAdapter<Object>
             topVer = cctx.mvcc().lastExplicitLockTopologyVersion(Thread.currentThread().getId());
 
         if (topVer == null)
-            mapOnTopology(null, false, null, waitTopFut);
+            mapOnTopology(null, false, null);
         else {
             topLocked = true;
 
+            // Cannot remap.
+            remapCnt.set(1);
+
             map0(topVer, null, false, null);
         }
     }
 
+    /**
+     * @param failed Keys to remap.
+     */
+    private void remap(Collection<?> failed) {
+        if (futVer != null)
+            cctx.mvcc().removeAtomicFuture(version());
+
+        Collection<Object> remapKeys = new ArrayList<>(failed.size());
+        Collection<Object> remapVals = new ArrayList<>(failed.size());
+
+        Iterator<?> keyIt = keys.iterator();
+        Iterator<?> valsIt = vals.iterator();
+
+        for (Object key : failed) {
+            while (keyIt.hasNext()) {
+                Object nextKey = keyIt.next();
+                Object nextVal = valsIt.next();
+
+                if (F.eq(key, nextKey)) {
+                    remapKeys.add(nextKey);
+                    remapVals.add(nextVal);
+
+                    break;
+                }
+            }
+        }
+
+        keys = remapKeys;
+        vals = remapVals;
+
+        mappings = new ConcurrentHashMap8<>(keys.size(), 1.0f);
+        single = null;
+        futVer = null;
+        err = null;
+        opRes = null;
+        topVer = AffinityTopologyVersion.ZERO;
+        singleNodeId = null;
+        singleReq = null;
+        fastMapRemap = false;
+        updVer = null;
+        topLocked = false;
+
+        map();
+    }
+
     /** {@inheritDoc} */
     @SuppressWarnings("ConstantConditions")
     @Override public boolean onDone(@Nullable Object res, @Nullable Throwable err) {
@@ -331,6 +388,12 @@ public class GridNearAtomicUpdateFuture extends GridFutureAdapter<Object>
         if (op == TRANSFORM && retval == null)
             retval = Collections.emptyMap();
 
+        if (err != null && X.hasCause(err, CachePartialUpdateCheckedException.class) && remapCnt.decrementAndGet() > 0) {
+            remap(X.cause(err, CachePartialUpdateCheckedException.class).failedKeys());
+
+            return false;
+        }
+
         if (super.onDone(retval, err)) {
             if (futVer != null)
                 cctx.mvcc().removeAtomicFuture(version());
@@ -353,7 +416,7 @@ public class GridNearAtomicUpdateFuture extends GridFutureAdapter<Object>
 
             Collection<KeyCacheObject> remapKeys = fastMap ? null : res.remapKeys();
 
-            mapOnTopology(remapKeys, true, nodeId, true);
+            mapOnTopology(remapKeys, true, nodeId);
 
             return;
         }
@@ -431,10 +494,8 @@ public class GridNearAtomicUpdateFuture extends GridFutureAdapter<Object>
      * @param keys Keys to map.
      * @param remap Boolean flag indicating if this is partial future remap.
      * @param oldNodeId Old node ID if remap.
-     * @param waitTopFut Whether to wait for topology future.
      */
-    private void mapOnTopology(final Collection<?> keys, final boolean remap, final UUID oldNodeId,
-        final boolean waitTopFut) {
+    private void mapOnTopology(final Collection<?> keys, final boolean remap, final UUID oldNodeId) {
         cache.topology().readLock();
 
         AffinityTopologyVersion topVer = null;
@@ -465,7 +526,7 @@ public class GridNearAtomicUpdateFuture extends GridFutureAdapter<Object>
                         @Override public void apply(IgniteInternalFuture<AffinityTopologyVersion> t) {
                             cctx.kernalContext().closure().runLocalSafe(new Runnable() {
                                 @Override public void run() {
-                                    mapOnTopology(keys, remap, oldNodeId, waitTopFut);
+                                    mapOnTopology(keys, remap, oldNodeId);
                                 }
                             });
                         }
@@ -509,7 +570,7 @@ public class GridNearAtomicUpdateFuture extends GridFutureAdapter<Object>
         }
 
         if (remap)
-            mapOnTopology(null, true, null, true);
+            mapOnTopology(null, true, null);
     }
 
     /**

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/3787a9d3/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/IgniteCachePutRetryAbstractSelfTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/IgniteCachePutRetryAbstractSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/IgniteCachePutRetryAbstractSelfTest.java
new file mode 100644
index 0000000..89d1040
--- /dev/null
+++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/IgniteCachePutRetryAbstractSelfTest.java
@@ -0,0 +1,134 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License.  You may obtain a copy of the License at
+ *
+ *      http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.ignite.internal.processors.cache.distributed.dht;
+
+import org.apache.ignite.cache.*;
+import org.apache.ignite.configuration.*;
+import org.apache.ignite.internal.*;
+import org.apache.ignite.internal.cluster.*;
+import org.apache.ignite.internal.processors.cache.*;
+import org.apache.ignite.internal.util.typedef.*;
+import org.apache.ignite.internal.util.typedef.internal.*;
+import org.apache.ignite.testframework.*;
+
+import java.util.concurrent.*;
+import java.util.concurrent.atomic.*;
+
+/**
+ *
+ */
+public abstract class IgniteCachePutRetryAbstractSelfTest extends GridCacheAbstractSelfTest {
+    /** {@inheritDoc} */
+    @Override protected int gridCount() {
+        return 4;
+    }
+
+    /**
+     * @return Keys count for the test.
+     */
+    protected abstract int keysCount();
+
+    /** {@inheritDoc} */
+    @Override protected CacheConfiguration cacheConfiguration(String gridName) throws Exception {
+        CacheConfiguration cfg = super.cacheConfiguration(gridName);
+
+        cfg.setBackups(1);
+
+        return cfg;
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testPut() throws Exception {
+        final AtomicBoolean finished = new AtomicBoolean();
+
+        IgniteInternalFuture<Object> fut = GridTestUtils.runAsync(new Callable<Object>() {
+            @Override public Object call() throws Exception {
+                while (!finished.get()) {
+                    stopGrid(3);
+
+                    U.sleep(300);
+
+                    startGrid(3);
+                }
+
+                return null;
+            }
+        });
+
+        int keysCnt = keysCount();
+
+        for (int i = 0; i < keysCnt; i++)
+            ignite(0).cache(null).put(i, i);
+
+        finished.set(true);
+        fut.get();
+
+        for (int i = 0; i < keysCnt; i++)
+            assertEquals(i, ignite(0).cache(null).get(i));
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testFailWithNoRetries() throws Exception {
+        final AtomicBoolean finished = new AtomicBoolean();
+
+        IgniteInternalFuture<Object> fut = GridTestUtils.runAsync(new Callable<Object>() {
+            @Override public Object call() throws Exception {
+                while (!finished.get()) {
+                    stopGrid(3);
+
+                    U.sleep(300);
+
+                    startGrid(3);
+                }
+
+                return null;
+            }
+        });
+
+        int keysCnt = keysCount();
+
+        boolean exceptionThrown = false;
+
+        for (int i = 0; i < keysCnt; i++) {
+            try {
+                ignite(0).cache(null).withNoRetries().put(i, i);
+            }
+            catch (Exception e) {
+                assertTrue("Invalid exception: " + e, X.hasCause(e, ClusterTopologyCheckedException.class) || X.hasCause(e, CachePartialUpdateException.class));
+
+                exceptionThrown = true;
+
+                break;
+            }
+        }
+
+        assertTrue(exceptionThrown);
+
+        finished.set(true);
+        fut.get();
+    }
+
+    /** {@inheritDoc} */
+    @Override protected long getTestTimeout() {
+        return 3 * 60 * 1000;
+    }
+}

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/3787a9d3/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/IgniteCachePutRetryAtomicSelfTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/IgniteCachePutRetryAtomicSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/IgniteCachePutRetryAtomicSelfTest.java
new file mode 100644
index 0000000..e76663a
--- /dev/null
+++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/IgniteCachePutRetryAtomicSelfTest.java
@@ -0,0 +1,34 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License.  You may obtain a copy of the License at
+ *
+ *      http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.ignite.internal.processors.cache.distributed.dht;
+
+import org.apache.ignite.cache.*;
+
+/**
+ *
+ */
+public class IgniteCachePutRetryAtomicSelfTest extends IgniteCachePutRetryAbstractSelfTest {
+    /** {@inheritDoc} */
+    @Override protected CacheAtomicityMode atomicityMode() {
+        return CacheAtomicityMode.ATOMIC;
+    }
+
+    /** {@inheritDoc} */
+    @Override protected int keysCount() {
+        return 60_000;
+    }
+}

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/3787a9d3/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/IgniteCachePutRetryTransactionalSelfTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/IgniteCachePutRetryTransactionalSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/IgniteCachePutRetryTransactionalSelfTest.java
new file mode 100644
index 0000000..e65459a
--- /dev/null
+++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/IgniteCachePutRetryTransactionalSelfTest.java
@@ -0,0 +1,35 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License.  You may obtain a copy of the License at
+ *
+ *      http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.ignite.internal.processors.cache.distributed.dht;
+
+import org.apache.ignite.cache.*;
+
+/**
+ *
+ */
+public class IgniteCachePutRetryTransactionalSelfTest extends IgniteCachePutRetryAbstractSelfTest {
+    /** {@inheritDoc} */
+    @Override protected CacheAtomicityMode atomicityMode() {
+        return CacheAtomicityMode.TRANSACTIONAL;
+    }
+
+    /** {@inheritDoc} */
+    @Override protected int keysCount() {
+        return 20_000;
+    }
+}


[06/13] incubator-ignite git commit: IGNITE-621 - Fixing remap logic.

Posted by ag...@apache.org.
IGNITE-621 - Fixing remap logic.


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

Branch: refs/heads/ignite-sprint-7
Commit: c2c90b52972bec53919d97ec07d2aeab4d0d55e8
Parents: a9d0662
Author: Alexey Goncharuk <ag...@gridgain.com>
Authored: Thu Jun 25 17:06:31 2015 -0700
Committer: Alexey Goncharuk <ag...@gridgain.com>
Committed: Thu Jun 25 17:06:31 2015 -0700

----------------------------------------------------------------------
 .../apache/ignite/IgniteSystemProperties.java   |  3 +
 .../processors/cache/GridCacheAdapter.java      |  2 +-
 .../processors/cache/GridCacheAtomicFuture.java | 12 ++-
 .../processors/cache/GridCacheMvccManager.java  |  8 +-
 .../dht/atomic/GridDhtAtomicUpdateFuture.java   | 12 ++-
 .../dht/atomic/GridNearAtomicUpdateFuture.java  | 88 ++++++++++++++++++--
 .../communication/tcp/TcpCommunicationSpi.java  |  2 +-
 ...eAtomicInvalidPartitionHandlingSelfTest.java |  7 +-
 8 files changed, 110 insertions(+), 24 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/c2c90b52/modules/core/src/main/java/org/apache/ignite/IgniteSystemProperties.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/IgniteSystemProperties.java b/modules/core/src/main/java/org/apache/ignite/IgniteSystemProperties.java
index 542fa30..40fc873 100644
--- a/modules/core/src/main/java/org/apache/ignite/IgniteSystemProperties.java
+++ b/modules/core/src/main/java/org/apache/ignite/IgniteSystemProperties.java
@@ -343,6 +343,9 @@ public final class IgniteSystemProperties {
     /** Maximum size for affinity assignment history. */
     public static final String IGNITE_AFFINITY_HISTORY_SIZE = "IGNITE_AFFINITY_HISTORY_SIZE";
 
+    /** Number of cache operation retries in case of topology exceptions. */
+    public static final String IGNITE_CACHE_RETRIES_COUNT = "IGNITE_CACHE_RETRIES_COUNT";
+
     /**
      * Enforces singleton.
      */

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/c2c90b52/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 f993527..e138520 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
@@ -79,7 +79,7 @@ public abstract class GridCacheAdapter<K, V> implements IgniteInternalCache<K, V
     public static final int CLEAR_ALL_SPLIT_THRESHOLD = 10000;
 
     /** Maximum number of retries when topology changes. */
-    public static final int MAX_RETRIES = 100;
+    public static final int MAX_RETRIES = IgniteSystemProperties.getInteger(IGNITE_CACHE_RETRIES_COUNT, 100);
 
     /** Deserialization stash. */
     private static final ThreadLocal<IgniteBiTuple<String, String>> stash = new ThreadLocal<IgniteBiTuple<String,

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/c2c90b52/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheAtomicFuture.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheAtomicFuture.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheAtomicFuture.java
index 35d3ec5..8724d3a 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheAtomicFuture.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheAtomicFuture.java
@@ -17,6 +17,7 @@
 
 package org.apache.ignite.internal.processors.cache;
 
+import org.apache.ignite.internal.*;
 import org.apache.ignite.internal.processors.affinity.*;
 
 import java.util.*;
@@ -26,14 +27,17 @@ import java.util.*;
  */
 public interface GridCacheAtomicFuture<R> extends GridCacheFuture<R> {
     /**
-     * @return {@code True} if partition exchange should wait for this future to complete.
+     * @return Future topology version.
      */
-    public boolean waitForPartitionExchange();
+    public AffinityTopologyVersion topologyVersion();
 
     /**
-     * @return Future topology version.
+     * Gets future that will be completed when it is safe when update is finished on the given version of topology.
+     *
+     * @param topVer Topology version to finish.
+     * @return Future or {@code null} if no need to wait.
      */
-    public AffinityTopologyVersion topologyVersion();
+    public IgniteInternalFuture<Void> completeFuture(AffinityTopologyVersion topVer);
 
     /**
      * @return Future keys.

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/c2c90b52/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheMvccManager.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheMvccManager.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheMvccManager.java
index c528e08..f24cf01 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheMvccManager.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheMvccManager.java
@@ -338,7 +338,7 @@ public class GridCacheMvccManager extends GridCacheSharedManagerAdapter {
     public void addAtomicFuture(GridCacheVersion futVer, GridCacheAtomicFuture<?> fut) {
         IgniteInternalFuture<?> old = atomicFuts.put(futVer, fut);
 
-        assert old == null;
+        assert old == null : "Old future is not null [futVer=" + futVer + ", fut=" + fut + ", old=" + old + ']';
     }
 
     /**
@@ -1002,8 +1002,10 @@ public class GridCacheMvccManager extends GridCacheSharedManagerAdapter {
         res.ignoreChildFailures(ClusterTopologyCheckedException.class, CachePartialUpdateCheckedException.class);
 
         for (GridCacheAtomicFuture<?> fut : atomicFuts.values()) {
-            if (fut.waitForPartitionExchange() && fut.topologyVersion().compareTo(topVer) < 0)
-                res.add((IgniteInternalFuture<Object>)fut);
+            IgniteInternalFuture<Void> complete = fut.completeFuture(topVer);
+
+            if (complete != null)
+                res.add((IgniteInternalFuture)complete);
         }
 
         res.markInitialized();

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/c2c90b52/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridDhtAtomicUpdateFuture.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridDhtAtomicUpdateFuture.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridDhtAtomicUpdateFuture.java
index ff8454e..37b57e6 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridDhtAtomicUpdateFuture.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridDhtAtomicUpdateFuture.java
@@ -20,6 +20,7 @@ package org.apache.ignite.internal.processors.cache.distributed.dht.atomic;
 import org.apache.ignite.*;
 import org.apache.ignite.cache.*;
 import org.apache.ignite.cluster.*;
+import org.apache.ignite.internal.*;
 import org.apache.ignite.internal.cluster.*;
 import org.apache.ignite.internal.processors.affinity.*;
 import org.apache.ignite.internal.processors.cache.*;
@@ -170,13 +171,16 @@ public class GridDhtAtomicUpdateFuture extends GridFutureAdapter<Void>
     }
 
     /** {@inheritDoc} */
-    @Override public boolean waitForPartitionExchange() {
-        return waitForExchange;
+    @Override public AffinityTopologyVersion topologyVersion() {
+        return updateReq.topologyVersion();
     }
 
     /** {@inheritDoc} */
-    @Override public AffinityTopologyVersion topologyVersion() {
-        return updateReq.topologyVersion();
+    @Override public IgniteInternalFuture<Void> completeFuture(AffinityTopologyVersion topVer) {
+        if (waitForExchange && topologyVersion().compareTo(topVer) < 0)
+            return this;
+
+        return null;
     }
 
     /** {@inheritDoc} */

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/c2c90b52/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridNearAtomicUpdateFuture.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridNearAtomicUpdateFuture.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridNearAtomicUpdateFuture.java
index 536eb40..ea9b335 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridNearAtomicUpdateFuture.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridNearAtomicUpdateFuture.java
@@ -105,7 +105,10 @@ public class GridNearAtomicUpdateFuture extends GridFutureAdapter<Object>
     private final ExpiryPolicy expiryPlc;
 
     /** Future map topology version. */
-    private AffinityTopologyVersion topVer = AffinityTopologyVersion.ZERO;
+    private volatile AffinityTopologyVersion topVer = AffinityTopologyVersion.ZERO;
+
+    /** Completion future for a particular topology version. */
+    private GridFutureAdapter<Void> topCompleteFut;
 
     /** Optional filter. */
     private final CacheEntryPredicate[] filter;
@@ -246,8 +249,10 @@ public class GridNearAtomicUpdateFuture extends GridFutureAdapter<Object>
         return F.view(F.viewReadOnly(mappings.keySet(), U.id2Node(cctx.kernalContext())), F.notNull());
     }
 
-    /** {@inheritDoc} */
-    @Override public boolean waitForPartitionExchange() {
+    /**
+     * @return {@code True} if this future should block partition map exchange.
+     */
+    private boolean waitForPartitionExchange() {
         // Wait fast-map near atomic update futures in CLOCK mode.
         return fastMap;
     }
@@ -323,13 +328,36 @@ public class GridNearAtomicUpdateFuture extends GridFutureAdapter<Object>
         else {
             topLocked = true;
 
-            // Cannot remap.
-            remapCnt.set(1);
+            synchronized (this) {
+                this.topVer = topVer;
+
+                // Cannot remap.
+                remapCnt.set(1);
+            }
 
             map0(topVer, null, false, null);
         }
     }
 
+    /** {@inheritDoc} */
+    @Override public IgniteInternalFuture<Void> completeFuture(AffinityTopologyVersion topVer) {
+        if (waitForPartitionExchange() && topologyVersion().compareTo(topVer) < 0) {
+            synchronized (this) {
+                if (this.topVer == AffinityTopologyVersion.ZERO)
+                    return null;
+
+                if (this.topVer.compareTo(topVer) < 0) {
+                    if (topCompleteFut == null)
+                        topCompleteFut = new GridFutureAdapter<>();
+
+                    return topCompleteFut;
+                }
+            }
+        }
+
+        return null;
+    }
+
     /**
      * @param failed Keys to remap.
      */
@@ -339,14 +367,20 @@ public class GridNearAtomicUpdateFuture extends GridFutureAdapter<Object>
 
         Collection<Object> remapKeys = new ArrayList<>(failed.size());
         Collection<Object> remapVals = vals != null ? new ArrayList<>(failed.size()) : null;
+        Collection<GridCacheDrInfo> remapConflictPutVals = conflictPutVals != null ? new ArrayList<GridCacheDrInfo>(failed.size()) : null;
+        Collection<GridCacheVersion> remapConflictRmvVals = conflictRmvVals != null ? new ArrayList<GridCacheVersion>(failed.size()) : null;
 
         Iterator<?> keyIt = keys.iterator();
         Iterator<?> valsIt = vals != null ? vals.iterator() : null;
+        Iterator<GridCacheDrInfo> conflictPutValsIt = conflictPutVals != null ? conflictPutVals.iterator() : null;
+        Iterator<GridCacheVersion> conflictRmvValsIt = conflictRmvVals != null ? conflictRmvVals.iterator() : null;
 
         for (Object key : failed) {
             while (keyIt.hasNext()) {
                 Object nextKey = keyIt.next();
                 Object nextVal = valsIt != null ? valsIt.next() : null;
+                GridCacheDrInfo nextConflictPutVal = conflictPutValsIt != null ? conflictPutValsIt.next() : null;
+                GridCacheVersion nextConflictRmvVal = conflictRmvValsIt != null ? conflictRmvValsIt.next() : null;
 
                 if (F.eq(key, nextKey)) {
                     remapKeys.add(nextKey);
@@ -354,6 +388,12 @@ public class GridNearAtomicUpdateFuture extends GridFutureAdapter<Object>
                     if (remapVals != null)
                         remapVals.add(nextVal);
 
+                    if (remapConflictPutVals != null)
+                        remapConflictPutVals.add(nextConflictPutVal);
+
+                    if (remapConflictRmvVals != null)
+                        remapConflictRmvVals.add(nextConflictRmvVal);
+
                     break;
                 }
             }
@@ -361,13 +401,29 @@ public class GridNearAtomicUpdateFuture extends GridFutureAdapter<Object>
 
         keys = remapKeys;
         vals = remapVals;
+        conflictPutVals = remapConflictPutVals;
+        conflictRmvVals = remapConflictRmvVals;
 
-        mappings = new ConcurrentHashMap8<>(keys.size(), 1.0f);
         single = null;
         futVer = null;
         err = null;
         opRes = null;
-        topVer = AffinityTopologyVersion.ZERO;
+
+        GridFutureAdapter<Void> fut0;
+
+        synchronized (this) {
+            mappings = new ConcurrentHashMap8<>(keys.size(), 1.0f);
+
+            topVer = AffinityTopologyVersion.ZERO;
+
+            fut0 = topCompleteFut;
+
+            topCompleteFut = null;
+        }
+
+        if (fut0 != null)
+            fut0.onDone();
+
         singleNodeId = null;
         singleReq = null;
         fastMapRemap = false;
@@ -405,6 +461,15 @@ public class GridNearAtomicUpdateFuture extends GridFutureAdapter<Object>
             if (futVer != null)
                 cctx.mvcc().removeAtomicFuture(version());
 
+            GridFutureAdapter<Void> fut0;
+
+            synchronized (this) {
+                fut0 = topCompleteFut;
+            }
+
+            if (fut0 != null)
+                fut0.onDone();
+
             return true;
         }
 
@@ -544,6 +609,10 @@ public class GridNearAtomicUpdateFuture extends GridFutureAdapter<Object>
 
                 return;
             }
+
+            synchronized (this) {
+                this.topVer = topVer;
+            }
         }
         finally {
             cache.topology().readUnlock();
@@ -559,7 +628,8 @@ public class GridNearAtomicUpdateFuture extends GridFutureAdapter<Object>
         boolean remap = false;
 
         synchronized (this) {
-            if ((syncMode == FULL_ASYNC && cctx.config().getAtomicWriteOrderMode() == PRIMARY) || mappings.isEmpty()) {
+            if (topVer != AffinityTopologyVersion.ZERO &&
+                ((syncMode == FULL_ASYNC && cctx.config().getAtomicWriteOrderMode() == PRIMARY) || mappings.isEmpty())) {
                 CachePartialUpdateCheckedException err0 = err;
 
                 if (err0 != null)
@@ -1040,7 +1110,7 @@ public class GridNearAtomicUpdateFuture extends GridFutureAdapter<Object>
         if (err0 == null)
             err0 = this.err = new CachePartialUpdateCheckedException("Failed to update keys (retry update if possible).");
 
-        List<Object> keys = new ArrayList<>(failedKeys.size());
+        Collection<Object> keys = new ArrayList<>(failedKeys.size());
 
         for (KeyCacheObject key : failedKeys)
             keys.add(key.value(cctx.cacheObjectContext(), false));

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/c2c90b52/modules/core/src/main/java/org/apache/ignite/spi/communication/tcp/TcpCommunicationSpi.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/spi/communication/tcp/TcpCommunicationSpi.java b/modules/core/src/main/java/org/apache/ignite/spi/communication/tcp/TcpCommunicationSpi.java
index addf243d..4ca2995 100644
--- a/modules/core/src/main/java/org/apache/ignite/spi/communication/tcp/TcpCommunicationSpi.java
+++ b/modules/core/src/main/java/org/apache/ignite/spi/communication/tcp/TcpCommunicationSpi.java
@@ -210,7 +210,7 @@ public class TcpCommunicationSpi extends IgniteSpiAdapter
     public static final int DFLT_ACK_SND_THRESHOLD = 16;
 
     /** Default socket write timeout. */
-    public static final long DFLT_SOCK_WRITE_TIMEOUT = 200;
+    public static final long DFLT_SOCK_WRITE_TIMEOUT = 2000;
 
     /** No-op runnable. */
     private static final IgniteRunnable NOOP = new IgniteRunnable() {

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/c2c90b52/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridCacheAtomicInvalidPartitionHandlingSelfTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridCacheAtomicInvalidPartitionHandlingSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridCacheAtomicInvalidPartitionHandlingSelfTest.java
index 054a110..b255558 100644
--- a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridCacheAtomicInvalidPartitionHandlingSelfTest.java
+++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridCacheAtomicInvalidPartitionHandlingSelfTest.java
@@ -41,6 +41,7 @@ import org.jsr166.*;
 import java.util.*;
 import java.util.concurrent.*;
 import java.util.concurrent.atomic.*;
+import java.util.concurrent.locks.*;
 
 import static org.apache.ignite.cache.CacheAtomicWriteOrderMode.*;
 import static org.apache.ignite.cache.CacheMode.*;
@@ -236,6 +237,8 @@ public class GridCacheAtomicInvalidPartitionHandlingSelfTest extends GridCommonA
 
             System.err.println("FINISHED PUTS");
 
+            GridCacheMapEntry.debug = true;
+
             // Start put threads.
             IgniteInternalFuture<?> fut = multithreadedAsync(new Callable<Object>() {
                 @Override public Object call() throws Exception {
@@ -340,12 +343,12 @@ public class GridCacheAtomicInvalidPartitionHandlingSelfTest extends GridCommonA
                         }
                         catch (AssertionError e) {
                             if (r == 9) {
-                                System.err.println("Failed to verify cache contents: " + e.getMessage());
+                                info("Failed to verify cache contents: " + e.getMessage());
 
                                 throw e;
                             }
 
-                            System.err.println("Failed to verify cache contents, will retry: " + e.getMessage());
+                            info("Failed to verify cache contents, will retry: " + e.getMessage());
 
                             // Give some time to finish async updates.
                             U.sleep(1000);


[13/13] incubator-ignite git commit: Merge remote-tracking branch 'origin/ignite-sprint-7' into ignite-sprint-7

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


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

Branch: refs/heads/ignite-sprint-7
Commit: ea90d8633265723076f729b2dca02873c033ce55
Parents: 1427d75 a0a31e2
Author: Alexey Goncharuk <ag...@gridgain.com>
Authored: Thu Jul 2 12:10:38 2015 -0700
Committer: Alexey Goncharuk <ag...@gridgain.com>
Committed: Thu Jul 2 12:10:38 2015 -0700

----------------------------------------------------------------------
 .../processors/cache/GridCacheSwapManager.java  | 257 ++++++++++++-------
 .../inmemory/GridTestSwapSpaceSpi.java          |   3 +-
 .../query/h2/opt/GridH2KeyValueRowOffheap.java  |   8 +-
 .../processors/query/h2/opt/GridH2Table.java    |   2 +-
 .../cache/IgniteCacheOffheapEvictQueryTest.java |   2 +-
 .../IgniteCacheQueryMultiThreadedSelfTest.java  |   4 +-
 ...QueryOffheapEvictsMultiThreadedSelfTest.java |   5 -
 .../IgniteCacheQueryNodeRestartSelfTest2.java   |   5 +
 8 files changed, 179 insertions(+), 107 deletions(-)
----------------------------------------------------------------------



[07/13] incubator-ignite git commit: Merge branch 'ignite-sprint-7' of https://git-wip-us.apache.org/repos/asf/incubator-ignite into ignite-621

Posted by ag...@apache.org.
Merge branch 'ignite-sprint-7' of https://git-wip-us.apache.org/repos/asf/incubator-ignite into ignite-621


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

Branch: refs/heads/ignite-sprint-7
Commit: 45b0f0925fe1da0fb41c06a2dce6440a26490f93
Parents: c2c90b5 3e8ddb4
Author: Alexey Goncharuk <ag...@gridgain.com>
Authored: Thu Jun 25 17:06:46 2015 -0700
Committer: Alexey Goncharuk <ag...@gridgain.com>
Committed: Thu Jun 25 17:06:46 2015 -0700

----------------------------------------------------------------------
 .../ClientAbstractConnectivitySelfTest.java     |   4 +-
 .../org/apache/ignite/cluster/ClusterGroup.java |   9 +
 .../org/apache/ignite/cluster/ClusterNode.java  |   2 +
 .../ignite/compute/ComputeTaskSplitAdapter.java |   2 +-
 .../ignite/internal/GridKernalContextImpl.java  |   3 +
 .../internal/cluster/ClusterGroupAdapter.java   |  38 +
 .../cluster/IgniteClusterAsyncImpl.java         |   5 +
 .../processors/plugin/CachePluginManager.java   |  10 +-
 .../internal/util/GridConfigurationFinder.java  |  55 +-
 .../ignite/internal/util/IgniteUtils.java       |   6 +-
 .../internal/ClusterForHostsSelfTest.java       | 113 +++
 .../internal/ClusterGroupAbstractTest.java      | 777 ++++++++++++++++++
 .../ignite/internal/ClusterGroupSelfTest.java   | 251 ++++++
 .../internal/GridProjectionAbstractTest.java    | 784 -------------------
 .../ignite/internal/GridProjectionSelfTest.java | 251 ------
 .../apache/ignite/internal/GridSelfTest.java    |   2 +-
 .../internal/util/IgniteUtilsSelfTest.java      |  22 +
 .../ignite/testsuites/IgniteBasicTestSuite.java |   6 +-
 .../p2p/GridP2PContinuousDeploymentTask1.java   |   2 +-
 .../commands/cache/VisorCacheCommand.scala      |   7 +-
 scripts/git-patch-prop.sh                       |   2 +-
 21 files changed, 1273 insertions(+), 1078 deletions(-)
----------------------------------------------------------------------



[09/13] incubator-ignite git commit: Merge branch 'ignite-sprint-7' of https://git-wip-us.apache.org/repos/asf/incubator-ignite into ignite-621

Posted by ag...@apache.org.
Merge branch 'ignite-sprint-7' of https://git-wip-us.apache.org/repos/asf/incubator-ignite into ignite-621


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

Branch: refs/heads/ignite-sprint-7
Commit: d24658d5a149da56f37a040b33ef12f3e9a3ad04
Parents: d8c1cc2 2ee616d
Author: Alexey Goncharuk <ag...@gridgain.com>
Authored: Fri Jun 26 14:25:25 2015 -0700
Committer: Alexey Goncharuk <ag...@gridgain.com>
Committed: Fri Jun 26 14:25:25 2015 -0700

----------------------------------------------------------------------
 .../org/apache/ignite/cluster/ClusterGroup.java |   9 +-
 .../internal/cluster/ClusterGroupAdapter.java   |  12 +-
 .../cluster/IgniteClusterAsyncImpl.java         |   7 +-
 .../GridCachePartitionExchangeManager.java      |   6 +-
 .../processors/rest/GridRestProcessor.java      |   4 +-
 .../handlers/task/GridTaskCommandHandler.java   |  12 +-
 .../processors/task/GridTaskWorker.java         |   4 +-
 .../visor/query/VisorQueryCleanupTask.java      |  14 +
 .../util/VisorClusterGroupEmptyException.java   |  33 +++
 .../ignite/spi/discovery/tcp/ClientImpl.java    | 151 ++++++-----
 .../ignite/spi/discovery/tcp/ServerImpl.java    | 103 +++++--
 .../spi/discovery/tcp/TcpDiscoverySpi.java      |   3 +-
 .../internal/ClusterForHostsSelfTest.java       | 113 --------
 .../internal/ClusterGroupHostsSelfTest.java     | 141 ++++++++++
 .../internal/GridDiscoveryEventSelfTest.java    |  12 +-
 ...achePartitionedMultiNodeFullApiSelfTest.java |   4 +-
 .../tcp/TcpClientDiscoverySpiSelfTest.java      | 265 ++++++++++++++++++-
 .../ignite/testsuites/IgniteBasicTestSuite.java |   2 +-
 18 files changed, 684 insertions(+), 211 deletions(-)
----------------------------------------------------------------------



[02/13] incubator-ignite git commit: IGNITE-621 - Added automatic retries for atomics.

Posted by ag...@apache.org.
IGNITE-621 - Added automatic retries for atomics.


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

Branch: refs/heads/ignite-sprint-7
Commit: 5505b4d3d50caf41246d1194c52298a6df47a239
Parents: 3787a9d
Author: Alexey Goncharuk <ag...@gridgain.com>
Authored: Sun Jun 21 13:09:24 2015 -0700
Committer: Alexey Goncharuk <ag...@gridgain.com>
Committed: Sun Jun 21 13:09:24 2015 -0700

----------------------------------------------------------------------
 .../processors/cache/GridCacheUtils.java        | 42 ++++++++++++++++++++
 .../datastructures/GridCacheAtomicLongImpl.java | 25 ++++++------
 .../GridCacheAtomicSequenceImpl.java            | 11 ++---
 .../GridCacheAtomicStampedImpl.java             | 21 +++++-----
 .../GridCacheCountDownLatchImpl.java            | 16 +++-----
 .../IgniteCachePutRetryAbstractSelfTest.java    | 13 ++++++
 ...gniteCachePutRetryTransactionalSelfTest.java | 39 ++++++++++++++++++
 .../IgniteCacheFailoverTestSuite.java           |  3 ++
 8 files changed, 131 insertions(+), 39 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/5505b4d3/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheUtils.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheUtils.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheUtils.java
index 8c26046..f88e288 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheUtils.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheUtils.java
@@ -30,6 +30,7 @@ import org.apache.ignite.internal.processors.cache.distributed.*;
 import org.apache.ignite.internal.processors.cache.distributed.dht.*;
 import org.apache.ignite.internal.processors.cache.transactions.*;
 import org.apache.ignite.internal.processors.cache.version.*;
+import org.apache.ignite.internal.transactions.*;
 import org.apache.ignite.internal.util.lang.*;
 import org.apache.ignite.internal.util.typedef.*;
 import org.apache.ignite.internal.util.typedef.T2;
@@ -1699,4 +1700,45 @@ public class GridCacheUtils {
             ctx.resource().cleanupGeneric(lsnr);
         }
     }
+
+    /**
+     * @param c Closure to retry.
+     * @param <S> Closure type.
+     * @return Wrapped closure.
+     */
+    public static <S> Callable<S> retryTopologySafe(final Callable<S> c ) {
+        return new Callable<S>() {
+            @Override public S call() throws Exception {
+                int retries = GridCacheAdapter.MAX_RETRIES;
+
+                IgniteCheckedException err = null;
+
+                for (int i = 0; i < retries; i++) {
+                    try {
+                        return c.call();
+                    }
+                    catch (IgniteCheckedException e) {
+                        if (X.hasCause(e, ClusterTopologyCheckedException.class) ||
+                            X.hasCause(e, IgniteTxRollbackCheckedException.class) ||
+                            X.hasCause(e, CachePartialUpdateCheckedException.class)) {
+                            if (i < retries - 1) {
+                                err = e;
+
+                                U.sleep(1);
+
+                                continue;
+                            }
+
+                            throw e;
+                        }
+                        else
+                            throw e;
+                    }
+                }
+
+                // Should never happen.
+                throw err;
+            }
+        };
+    }
 }

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/5505b4d3/modules/core/src/main/java/org/apache/ignite/internal/processors/datastructures/GridCacheAtomicLongImpl.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/datastructures/GridCacheAtomicLongImpl.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/datastructures/GridCacheAtomicLongImpl.java
index b18d35a..5e9245d 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/datastructures/GridCacheAtomicLongImpl.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/datastructures/GridCacheAtomicLongImpl.java
@@ -31,6 +31,7 @@ import java.util.concurrent.*;
 
 import static org.apache.ignite.transactions.TransactionConcurrency.*;
 import static org.apache.ignite.transactions.TransactionIsolation.*;
+import static org.apache.ignite.internal.util.typedef.internal.CU.*;
 
 /**
  * Cache atomic long implementation.
@@ -78,7 +79,7 @@ public final class GridCacheAtomicLongImpl implements GridCacheAtomicLongEx, Ext
     };
 
     /** Callable for {@link #incrementAndGet()}. */
-    private final Callable<Long> incAndGetCall = new Callable<Long>() {
+    private final Callable<Long> incAndGetCall = retryTopologySafe(new Callable<Long>() {
         @Override public Long call() throws Exception {
             try (IgniteInternalTx tx = CU.txStartInternal(ctx, atomicView, PESSIMISTIC, REPEATABLE_READ)) {
                 GridCacheAtomicLongValue val = atomicView.get(key);
@@ -102,10 +103,10 @@ public final class GridCacheAtomicLongImpl implements GridCacheAtomicLongEx, Ext
                 throw e;
             }
         }
-    };
+    });
 
     /** Callable for {@link #getAndIncrement()}. */
-    private final Callable<Long> getAndIncCall = new Callable<Long>() {
+    private final Callable<Long> getAndIncCall = retryTopologySafe(new Callable<Long>() {
         @Override public Long call() throws Exception {
             try (IgniteInternalTx tx = CU.txStartInternal(ctx, atomicView, PESSIMISTIC, REPEATABLE_READ)) {
                 GridCacheAtomicLongValue val = atomicView.get(key);
@@ -129,10 +130,10 @@ public final class GridCacheAtomicLongImpl implements GridCacheAtomicLongEx, Ext
                 throw e;
             }
         }
-    };
+    });
 
     /** Callable for {@link #decrementAndGet()}. */
-    private final Callable<Long> decAndGetCall = new Callable<Long>() {
+    private final Callable<Long> decAndGetCall = retryTopologySafe(new Callable<Long>() {
         @Override public Long call() throws Exception {
             try (IgniteInternalTx tx = CU.txStartInternal(ctx, atomicView, PESSIMISTIC, REPEATABLE_READ)) {
                 GridCacheAtomicLongValue val = atomicView.get(key);
@@ -156,10 +157,10 @@ public final class GridCacheAtomicLongImpl implements GridCacheAtomicLongEx, Ext
                 throw e;
             }
         }
-    };
+    });
 
     /** Callable for {@link #getAndDecrement()}. */
-    private final Callable<Long> getAndDecCall = new Callable<Long>() {
+    private final Callable<Long> getAndDecCall = retryTopologySafe(new Callable<Long>() {
         @Override public Long call() throws Exception {
             try (IgniteInternalTx tx = CU.txStartInternal(ctx, atomicView, PESSIMISTIC, REPEATABLE_READ)) {
                 GridCacheAtomicLongValue val = atomicView.get(key);
@@ -183,7 +184,7 @@ public final class GridCacheAtomicLongImpl implements GridCacheAtomicLongEx, Ext
                 throw e;
             }
         }
-    };
+    });
 
     /**
      * Empty constructor required by {@link Externalizable}.
@@ -378,7 +379,7 @@ public final class GridCacheAtomicLongImpl implements GridCacheAtomicLongEx, Ext
      * @return Callable for execution in async and sync mode.
      */
     private Callable<Long> internalAddAndGet(final long l) {
-        return new Callable<Long>() {
+        return retryTopologySafe(new Callable<Long>() {
             @Override public Long call() throws Exception {
                 try (IgniteInternalTx tx = CU.txStartInternal(ctx, atomicView, PESSIMISTIC, REPEATABLE_READ)) {
                     GridCacheAtomicLongValue val = atomicView.get(key);
@@ -402,7 +403,7 @@ public final class GridCacheAtomicLongImpl implements GridCacheAtomicLongEx, Ext
                     throw e;
                 }
             }
-        };
+        });
     }
 
     /**
@@ -412,7 +413,7 @@ public final class GridCacheAtomicLongImpl implements GridCacheAtomicLongEx, Ext
      * @return Callable for execution in async and sync mode.
      */
     private Callable<Long> internalGetAndAdd(final long l) {
-        return new Callable<Long>() {
+        return retryTopologySafe(new Callable<Long>() {
             @Override public Long call() throws Exception {
                 try (IgniteInternalTx tx = CU.txStartInternal(ctx, atomicView, PESSIMISTIC, REPEATABLE_READ)) {
                     GridCacheAtomicLongValue val = atomicView.get(key);
@@ -436,7 +437,7 @@ public final class GridCacheAtomicLongImpl implements GridCacheAtomicLongEx, Ext
                     throw e;
                 }
             }
-        };
+        });
     }
 
     /**

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/5505b4d3/modules/core/src/main/java/org/apache/ignite/internal/processors/datastructures/GridCacheAtomicSequenceImpl.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/datastructures/GridCacheAtomicSequenceImpl.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/datastructures/GridCacheAtomicSequenceImpl.java
index e66c11e..2400a7e 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/datastructures/GridCacheAtomicSequenceImpl.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/datastructures/GridCacheAtomicSequenceImpl.java
@@ -35,6 +35,7 @@ import java.util.concurrent.locks.*;
 import static java.util.concurrent.TimeUnit.*;
 import static org.apache.ignite.transactions.TransactionConcurrency.*;
 import static org.apache.ignite.transactions.TransactionIsolation.*;
+import static org.apache.ignite.internal.util.typedef.internal.CU.*;
 
 /**
  * Cache sequence implementation.
@@ -435,11 +436,9 @@ public final class GridCacheAtomicSequenceImpl implements GridCacheAtomicSequenc
      */
     @SuppressWarnings("TooBroadScope")
     private Callable<Long> internalUpdate(final long l, final boolean updated) {
-        return new Callable<Long>() {
+        return retryTopologySafe(new Callable<Long>() {
             @Override public Long call() throws Exception {
-                IgniteInternalTx tx = CU.txStartInternal(ctx, seqView, PESSIMISTIC, REPEATABLE_READ);
-
-                try {
+                try (IgniteInternalTx tx = CU.txStartInternal(ctx, seqView, PESSIMISTIC, REPEATABLE_READ)) {
                     GridCacheAtomicSequenceValue seq = seqView.get(key);
 
                     checkRemoved();
@@ -506,11 +505,9 @@ public final class GridCacheAtomicSequenceImpl implements GridCacheAtomicSequenc
                     U.error(log, "Failed to get and add: " + this, e);
 
                     throw e;
-                } finally {
-                    tx.close();
                 }
             }
-        };
+        });
     }
 
     /** {@inheritDoc} */

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/5505b4d3/modules/core/src/main/java/org/apache/ignite/internal/processors/datastructures/GridCacheAtomicStampedImpl.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/datastructures/GridCacheAtomicStampedImpl.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/datastructures/GridCacheAtomicStampedImpl.java
index a898e58..76ea7ca 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/datastructures/GridCacheAtomicStampedImpl.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/datastructures/GridCacheAtomicStampedImpl.java
@@ -33,6 +33,7 @@ import java.util.concurrent.*;
 
 import static org.apache.ignite.transactions.TransactionConcurrency.*;
 import static org.apache.ignite.transactions.TransactionIsolation.*;
+import static org.apache.ignite.internal.util.typedef.internal.CU.*;
 
 /**
  * Cache atomic stamped implementation.
@@ -68,7 +69,7 @@ public final class GridCacheAtomicStampedImpl<T, S> implements GridCacheAtomicSt
     private GridCacheContext ctx;
 
     /** Callable for {@link #get()} operation */
-    private final Callable<IgniteBiTuple<T, S>> getCall = new Callable<IgniteBiTuple<T, S>>() {
+    private final Callable<IgniteBiTuple<T, S>> getCall = retryTopologySafe(new Callable<IgniteBiTuple<T, S>>() {
         @Override public IgniteBiTuple<T, S> call() throws Exception {
             GridCacheAtomicStampedValue<T, S> stmp = atomicView.get(key);
 
@@ -77,10 +78,10 @@ public final class GridCacheAtomicStampedImpl<T, S> implements GridCacheAtomicSt
 
             return stmp.get();
         }
-    };
+    });
 
     /** Callable for {@link #value()} operation */
-    private final Callable<T> valCall = new Callable<T>() {
+    private final Callable<T> valCall = retryTopologySafe(new Callable<T>() {
         @Override public T call() throws Exception {
             GridCacheAtomicStampedValue<T, S> stmp = atomicView.get(key);
 
@@ -89,10 +90,10 @@ public final class GridCacheAtomicStampedImpl<T, S> implements GridCacheAtomicSt
 
             return stmp.value();
         }
-    };
+    });
 
     /** Callable for {@link #stamp()} operation */
-    private final Callable<S> stampCall = new Callable<S>() {
+    private final Callable<S> stampCall = retryTopologySafe(new Callable<S>() {
         @Override public S call() throws Exception {
             GridCacheAtomicStampedValue<T, S> stmp = atomicView.get(key);
 
@@ -101,7 +102,7 @@ public final class GridCacheAtomicStampedImpl<T, S> implements GridCacheAtomicSt
 
             return stmp.stamp();
         }
-    };
+    });
 
     /**
      * Empty constructor required by {@link Externalizable}.
@@ -254,7 +255,7 @@ public final class GridCacheAtomicStampedImpl<T, S> implements GridCacheAtomicSt
      * @return Callable for execution in async and sync mode.
      */
     private Callable<Boolean> internalSet(final T val, final S stamp) {
-        return new Callable<Boolean>() {
+        return retryTopologySafe(new Callable<Boolean>() {
             @Override public Boolean call() throws Exception {
                 try (IgniteInternalTx tx = CU.txStartInternal(ctx, atomicView, PESSIMISTIC, REPEATABLE_READ)) {
                     GridCacheAtomicStampedValue<T, S> stmp = atomicView.get(key);
@@ -276,7 +277,7 @@ public final class GridCacheAtomicStampedImpl<T, S> implements GridCacheAtomicSt
                     throw e;
                 }
             }
-        };
+        });
     }
 
     /**
@@ -292,7 +293,7 @@ public final class GridCacheAtomicStampedImpl<T, S> implements GridCacheAtomicSt
     private Callable<Boolean> internalCompareAndSet(final IgnitePredicate<T> expValPred,
         final IgniteClosure<T, T> newValClos, final IgnitePredicate<S> expStampPred,
         final IgniteClosure<S, S> newStampClos) {
-        return new Callable<Boolean>() {
+        return retryTopologySafe(new Callable<Boolean>() {
             @Override public Boolean call() throws Exception {
                 try (IgniteInternalTx tx = CU.txStartInternal(ctx, atomicView, PESSIMISTIC, REPEATABLE_READ)) {
                     GridCacheAtomicStampedValue<T, S> stmp = atomicView.get(key);
@@ -323,7 +324,7 @@ public final class GridCacheAtomicStampedImpl<T, S> implements GridCacheAtomicSt
                     throw e;
                 }
             }
-        };
+        });
     }
 
     /** {@inheritDoc} */

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/5505b4d3/modules/core/src/main/java/org/apache/ignite/internal/processors/datastructures/GridCacheCountDownLatchImpl.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/datastructures/GridCacheCountDownLatchImpl.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/datastructures/GridCacheCountDownLatchImpl.java
index 33547d9..ea7924f 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/datastructures/GridCacheCountDownLatchImpl.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/datastructures/GridCacheCountDownLatchImpl.java
@@ -32,6 +32,7 @@ import java.util.concurrent.atomic.*;
 
 import static org.apache.ignite.transactions.TransactionConcurrency.*;
 import static org.apache.ignite.transactions.TransactionIsolation.*;
+import static org.apache.ignite.internal.util.typedef.internal.CU.*;
 
 /**
  * Cache count down latch implementation.
@@ -179,12 +180,7 @@ public final class GridCacheCountDownLatchImpl implements GridCacheCountDownLatc
 
     /** {@inheritDoc} */
     @Override public int countDown() {
-        try {
-            return CU.outTx(new CountDownCallable(1), ctx);
-        }
-        catch (IgniteCheckedException e) {
-            throw U.convertException(e);
-        }
+        return countDown(1);
     }
 
     /** {@inheritDoc} */
@@ -192,7 +188,7 @@ public final class GridCacheCountDownLatchImpl implements GridCacheCountDownLatc
         A.ensure(val > 0, "val should be positive");
 
         try {
-            return CU.outTx(new CountDownCallable(val), ctx);
+            return CU.outTx(retryTopologySafe(new CountDownCallable(val)), ctx);
         }
         catch (IgniteCheckedException e) {
             throw U.convertException(e);
@@ -202,7 +198,7 @@ public final class GridCacheCountDownLatchImpl implements GridCacheCountDownLatc
     /** {@inheritDoc}*/
     @Override public void countDownAll() {
         try {
-            CU.outTx(new CountDownCallable(0), ctx);
+            CU.outTx(retryTopologySafe(new CountDownCallable(0)), ctx);
         }
         catch (IgniteCheckedException e) {
             throw U.convertException(e);
@@ -248,7 +244,7 @@ public final class GridCacheCountDownLatchImpl implements GridCacheCountDownLatc
         if (initGuard.compareAndSet(false, true)) {
             try {
                 internalLatch = CU.outTx(
-                    new Callable<CountDownLatch>() {
+                    retryTopologySafe(new Callable<CountDownLatch>() {
                         @Override public CountDownLatch call() throws Exception {
                             try (IgniteInternalTx tx = CU.txStartInternal(ctx, latchView, PESSIMISTIC, REPEATABLE_READ)) {
                                 GridCacheCountDownLatchValue val = latchView.get(key);
@@ -267,7 +263,7 @@ public final class GridCacheCountDownLatchImpl implements GridCacheCountDownLatc
                                 return new CountDownLatch(val.get());
                             }
                         }
-                    },
+                    }),
                     ctx
                 );
 

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/5505b4d3/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/IgniteCachePutRetryAbstractSelfTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/IgniteCachePutRetryAbstractSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/IgniteCachePutRetryAbstractSelfTest.java
index 89d1040..bfddbe7 100644
--- a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/IgniteCachePutRetryAbstractSelfTest.java
+++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/IgniteCachePutRetryAbstractSelfTest.java
@@ -52,6 +52,19 @@ public abstract class IgniteCachePutRetryAbstractSelfTest extends GridCacheAbstr
         return cfg;
     }
 
+    /** {@inheritDoc} */
+    @Override protected IgniteConfiguration getConfiguration(String gridName) throws Exception {
+        IgniteConfiguration cfg = super.getConfiguration(gridName);
+
+        AtomicConfiguration acfg = new AtomicConfiguration();
+
+        acfg.setBackups(1);
+
+        cfg.setAtomicConfiguration(acfg);
+
+        return cfg;
+    }
+
     /**
      * @throws Exception If failed.
      */

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/5505b4d3/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/IgniteCachePutRetryTransactionalSelfTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/IgniteCachePutRetryTransactionalSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/IgniteCachePutRetryTransactionalSelfTest.java
index e65459a..91c454a 100644
--- a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/IgniteCachePutRetryTransactionalSelfTest.java
+++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/IgniteCachePutRetryTransactionalSelfTest.java
@@ -17,7 +17,14 @@
 
 package org.apache.ignite.internal.processors.cache.distributed.dht;
 
+import org.apache.ignite.*;
 import org.apache.ignite.cache.*;
+import org.apache.ignite.internal.*;
+import org.apache.ignite.internal.util.typedef.internal.*;
+import org.apache.ignite.testframework.*;
+
+import java.util.concurrent.*;
+import java.util.concurrent.atomic.*;
 
 /**
  *
@@ -32,4 +39,36 @@ public class IgniteCachePutRetryTransactionalSelfTest extends IgniteCachePutRetr
     @Override protected int keysCount() {
         return 20_000;
     }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testAtomicLongRetries() throws Exception {
+        final AtomicBoolean finished = new AtomicBoolean();
+
+        IgniteAtomicLong atomic = ignite(0).atomicLong("TestAtomic", 0, true);
+
+        IgniteInternalFuture<Object> fut = GridTestUtils.runAsync(new Callable<Object>() {
+            @Override
+            public Object call() throws Exception {
+                while (!finished.get()) {
+                    stopGrid(3);
+
+                    U.sleep(300);
+
+                    startGrid(3);
+                }
+
+                return null;
+            }
+        });
+
+        int keysCnt = keysCount();
+
+        for (int i = 0; i < keysCnt; i++)
+            atomic.incrementAndGet();
+
+        finished.set(true);
+        fut.get();
+    }
 }

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/5505b4d3/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteCacheFailoverTestSuite.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteCacheFailoverTestSuite.java b/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteCacheFailoverTestSuite.java
index dda86c1..80bfbf2 100644
--- a/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteCacheFailoverTestSuite.java
+++ b/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteCacheFailoverTestSuite.java
@@ -72,6 +72,9 @@ public class IgniteCacheFailoverTestSuite extends TestSuite {
         suite.addTestSuite(IgniteCacheTxNearDisabledPutGetRestartTest.class);
         suite.addTestSuite(IgniteCacheTxNearDisabledFairAffinityPutGetRestartTest.class);
 
+        suite.addTestSuite(IgniteCachePutRetryAtomicSelfTest.class);
+        suite.addTestSuite(IgniteCachePutRetryTransactionalSelfTest.class);
+
         return suite;
     }
 }


[03/13] incubator-ignite git commit: IGNITE-621 - Retries.

Posted by ag...@apache.org.
IGNITE-621 - Retries.


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

Branch: refs/heads/ignite-sprint-7
Commit: c94c0c475b8d8ac5c31302235d0de36f791fc3a0
Parents: 5505b4d
Author: Alexey Goncharuk <ag...@gridgain.com>
Authored: Sun Jun 21 22:40:01 2015 -0700
Committer: Alexey Goncharuk <ag...@gridgain.com>
Committed: Sun Jun 21 22:40:01 2015 -0700

----------------------------------------------------------------------
 .../dht/atomic/GridNearAtomicUpdateFuture.java   | 19 +++++++++++++------
 1 file changed, 13 insertions(+), 6 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/c94c0c47/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridNearAtomicUpdateFuture.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridNearAtomicUpdateFuture.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridNearAtomicUpdateFuture.java
index 53150cc..536eb40 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridNearAtomicUpdateFuture.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridNearAtomicUpdateFuture.java
@@ -338,19 +338,21 @@ public class GridNearAtomicUpdateFuture extends GridFutureAdapter<Object>
             cctx.mvcc().removeAtomicFuture(version());
 
         Collection<Object> remapKeys = new ArrayList<>(failed.size());
-        Collection<Object> remapVals = new ArrayList<>(failed.size());
+        Collection<Object> remapVals = vals != null ? new ArrayList<>(failed.size()) : null;
 
         Iterator<?> keyIt = keys.iterator();
-        Iterator<?> valsIt = vals.iterator();
+        Iterator<?> valsIt = vals != null ? vals.iterator() : null;
 
         for (Object key : failed) {
             while (keyIt.hasNext()) {
                 Object nextKey = keyIt.next();
-                Object nextVal = valsIt.next();
+                Object nextVal = valsIt != null ? valsIt.next() : null;
 
                 if (F.eq(key, nextKey)) {
                     remapKeys.add(nextKey);
-                    remapVals.add(nextVal);
+
+                    if (remapVals != null)
+                        remapVals.add(nextVal);
 
                     break;
                 }
@@ -388,8 +390,13 @@ public class GridNearAtomicUpdateFuture extends GridFutureAdapter<Object>
         if (op == TRANSFORM && retval == null)
             retval = Collections.emptyMap();
 
-        if (err != null && X.hasCause(err, CachePartialUpdateCheckedException.class) && remapCnt.decrementAndGet() > 0) {
-            remap(X.cause(err, CachePartialUpdateCheckedException.class).failedKeys());
+        if (err != null && X.hasCause(err, CachePartialUpdateCheckedException.class) &&
+            X.hasCause(err, ClusterTopologyCheckedException.class) &&
+            remapCnt.decrementAndGet() > 0) {
+
+            CachePartialUpdateCheckedException cause = X.cause(err, CachePartialUpdateCheckedException.class);
+
+            remap(cause.failedKeys());
 
             return false;
         }


[11/13] incubator-ignite git commit: Merge branch 'ignite-sprint-7' of https://git-wip-us.apache.org/repos/asf/incubator-ignite into ignite-621

Posted by ag...@apache.org.
Merge branch 'ignite-sprint-7' of https://git-wip-us.apache.org/repos/asf/incubator-ignite into ignite-621


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

Branch: refs/heads/ignite-sprint-7
Commit: ac93958d21ef33d98bb223904131bf1bf54a7cf2
Parents: 0cb2d97 68c21ac
Author: Alexey Goncharuk <ag...@gridgain.com>
Authored: Mon Jun 29 17:40:01 2015 -0700
Committer: Alexey Goncharuk <ag...@gridgain.com>
Committed: Mon Jun 29 17:40:01 2015 -0700

----------------------------------------------------------------------
 .../ignite/internal/GridKernalContextImpl.java  |   2 +-
 .../discovery/GridDiscoveryManager.java         |   6 +
 .../cache/GridCacheDeploymentManager.java       |  10 +-
 .../processors/cache/GridCacheProcessor.java    |  62 ++++--
 .../shmem/IpcSharedMemoryServerEndpoint.java    |  10 +-
 .../apache/ignite/internal/visor/VisorJob.java  |   2 +
 .../internal/visor/log/VisorLogSearchTask.java  |   2 +-
 .../visor/node/VisorNodeDataCollectorJob.java   |   4 +
 .../util/VisorClusterGroupEmptyException.java   |   6 +-
 .../cache/GridCacheDaemonNodeStopSelfTest.java  | 119 ------------
 .../IgniteDaemonNodeMarshallerCacheTest.java    | 192 +++++++++++++++++++
 .../ignite/testsuites/IgniteBasicTestSuite.java |   1 +
 .../testsuites/IgniteCacheTestSuite3.java       |   1 -
 pom.xml                                         |  12 +-
 14 files changed, 282 insertions(+), 147 deletions(-)
----------------------------------------------------------------------



[10/13] incubator-ignite git commit: IGNITE-621 - Fixing tests.

Posted by ag...@apache.org.
IGNITE-621 - Fixing 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/0cb2d97c
Tree: http://git-wip-us.apache.org/repos/asf/incubator-ignite/tree/0cb2d97c
Diff: http://git-wip-us.apache.org/repos/asf/incubator-ignite/diff/0cb2d97c

Branch: refs/heads/ignite-sprint-7
Commit: 0cb2d97c61603017909ff6ffc4908f93b2384ec0
Parents: d24658d
Author: Alexey Goncharuk <ag...@gridgain.com>
Authored: Mon Jun 29 17:39:42 2015 -0700
Committer: Alexey Goncharuk <ag...@gridgain.com>
Committed: Mon Jun 29 17:39:42 2015 -0700

----------------------------------------------------------------------
 .../dht/atomic/GridDhtAtomicUpdateFuture.java   |  3 ---
 .../dht/atomic/GridNearAtomicUpdateFuture.java  | 23 ++++++++++----------
 2 files changed, 12 insertions(+), 14 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/0cb2d97c/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridDhtAtomicUpdateFuture.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridDhtAtomicUpdateFuture.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridDhtAtomicUpdateFuture.java
index 37b57e6..4b1a58f 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridDhtAtomicUpdateFuture.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridDhtAtomicUpdateFuture.java
@@ -146,9 +146,6 @@ public class GridDhtAtomicUpdateFuture extends GridFutureAdapter<Void>
         GridDhtAtomicUpdateRequest req = mappings.get(nodeId);
 
         if (req != null) {
-            updateRes.addFailedKeys(req.keys(), new ClusterTopologyCheckedException("Failed to write keys on backup " +
-                "(node left grid before response is received): " + nodeId));
-
             // Remove only after added keys to failed set.
             mappings.remove(nodeId);
 

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/0cb2d97c/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridNearAtomicUpdateFuture.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridNearAtomicUpdateFuture.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridNearAtomicUpdateFuture.java
index ea9b335..41cc400 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridNearAtomicUpdateFuture.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridNearAtomicUpdateFuture.java
@@ -328,12 +328,8 @@ public class GridNearAtomicUpdateFuture extends GridFutureAdapter<Object>
         else {
             topLocked = true;
 
-            synchronized (this) {
-                this.topVer = topVer;
-
-                // Cannot remap.
-                remapCnt.set(1);
-            }
+            // Cannot remap.
+            remapCnt.set(1);
 
             map0(topVer, null, false, null);
         }
@@ -452,6 +448,9 @@ public class GridNearAtomicUpdateFuture extends GridFutureAdapter<Object>
 
             CachePartialUpdateCheckedException cause = X.cause(err, CachePartialUpdateCheckedException.class);
 
+            if (F.isEmpty(cause.failedKeys()))
+                cause.printStackTrace();
+
             remap(cause.failedKeys());
 
             return false;
@@ -609,10 +608,6 @@ public class GridNearAtomicUpdateFuture extends GridFutureAdapter<Object>
 
                 return;
             }
-
-            synchronized (this) {
-                this.topVer = topVer;
-            }
         }
         finally {
             cache.topology().readUnlock();
@@ -786,7 +781,11 @@ public class GridNearAtomicUpdateFuture extends GridFutureAdapter<Object>
                 conflictVer,
                 true);
 
-            single = true;
+            synchronized (this) {
+                this.topVer = topVer;
+
+                single = true;
+            }
 
             // Optimize mapping for single key.
             mapSingle(primary.id(), req);
@@ -942,6 +941,8 @@ public class GridNearAtomicUpdateFuture extends GridFutureAdapter<Object>
                 }
             }
 
+            this.topVer = topVer;
+
             fastMapRemap = false;
         }
 


[08/13] incubator-ignite git commit: IGNITE-621 - Fixed compilation.

Posted by ag...@apache.org.
IGNITE-621 - Fixed compilation.


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

Branch: refs/heads/ignite-sprint-7
Commit: d8c1cc2f0160e80226b5e3f02130fc91843633d3
Parents: 45b0f09
Author: Alexey Goncharuk <ag...@gridgain.com>
Authored: Thu Jun 25 17:19:17 2015 -0700
Committer: Alexey Goncharuk <ag...@gridgain.com>
Committed: Thu Jun 25 17:19:17 2015 -0700

----------------------------------------------------------------------
 .../atomic/GridCacheAtomicInvalidPartitionHandlingSelfTest.java    | 2 --
 1 file changed, 2 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/d8c1cc2f/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridCacheAtomicInvalidPartitionHandlingSelfTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridCacheAtomicInvalidPartitionHandlingSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridCacheAtomicInvalidPartitionHandlingSelfTest.java
index b255558..8e69853 100644
--- a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridCacheAtomicInvalidPartitionHandlingSelfTest.java
+++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridCacheAtomicInvalidPartitionHandlingSelfTest.java
@@ -237,8 +237,6 @@ public class GridCacheAtomicInvalidPartitionHandlingSelfTest extends GridCommonA
 
             System.err.println("FINISHED PUTS");
 
-            GridCacheMapEntry.debug = true;
-
             // Start put threads.
             IgniteInternalFuture<?> fut = multithreadedAsync(new Callable<Object>() {
                 @Override public Object call() throws Exception {


[04/13] incubator-ignite git commit: Merge branch 'ignite-sprint-6' of https://git-wip-us.apache.org/repos/asf/incubator-ignite into ignite-621

Posted by ag...@apache.org.
Merge branch 'ignite-sprint-6' of https://git-wip-us.apache.org/repos/asf/incubator-ignite into ignite-621


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

Branch: refs/heads/ignite-sprint-7
Commit: a4e31be6a018f1711ec0cb385c6b88b35c9dffed
Parents: c94c0c4 285d790
Author: Alexey Goncharuk <ag...@gridgain.com>
Authored: Mon Jun 22 15:29:47 2015 -0700
Committer: Alexey Goncharuk <ag...@gridgain.com>
Committed: Mon Jun 22 15:29:47 2015 -0700

----------------------------------------------------------------------
 .../apache/ignite/IgniteSystemProperties.java   |   3 +
 .../ignite/internal/MarshallerContextImpl.java  |  12 +-
 .../processors/cache/GridCacheIoManager.java    |  64 ++++--
 .../GridCachePartitionExchangeManager.java      |  70 +++----
 .../processors/cache/GridCacheSwapManager.java  |  12 +-
 .../GridDhtPartitionsExchangeFuture.java        |  49 +++--
 .../offheap/GridOffHeapProcessor.java           |  19 +-
 .../apache/ignite/internal/util/GridDebug.java  |  37 ++--
 .../communication/tcp/TcpCommunicationSpi.java  |  61 +++---
 .../tcp/TcpCommunicationSpiMBean.java           |   8 +
 .../ignite/spi/discovery/tcp/ClientImpl.java    |  68 ++++---
 .../GridCacheAbstractFailoverSelfTest.java      |   6 +-
 .../cache/GridCacheDaemonNodeStopSelfTest.java  | 119 +++++++++++
 .../GridTcpCommunicationSpiConfigSelfTest.java  |   1 -
 .../testsuites/IgniteCacheTestSuite3.java       |   1 +
 .../processors/query/h2/IgniteH2Indexing.java   |   2 +
 .../query/h2/twostep/GridMapQueryExecutor.java  |  23 ++-
 .../cache/IgniteCacheOffheapEvictQueryTest.java | 196 +++++++++++++++++++
 .../IgniteCacheQuerySelfTestSuite.java          |   1 +
 19 files changed, 590 insertions(+), 162 deletions(-)
----------------------------------------------------------------------



[05/13] incubator-ignite git commit: Merge branch 'ignite-sprint-7' of https://git-wip-us.apache.org/repos/asf/incubator-ignite into ignite-621

Posted by ag...@apache.org.
Merge branch 'ignite-sprint-7' of https://git-wip-us.apache.org/repos/asf/incubator-ignite into ignite-621


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

Branch: refs/heads/ignite-sprint-7
Commit: a9d0662b1d084a770b1364295c48aa61e2c0dcfa
Parents: a4e31be b29ff1c
Author: Alexey Goncharuk <ag...@gridgain.com>
Authored: Wed Jun 24 15:25:40 2015 -0700
Committer: Alexey Goncharuk <ag...@gridgain.com>
Committed: Wed Jun 24 15:25:40 2015 -0700

----------------------------------------------------------------------
 examples/pom.xml                                |   2 +-
 modules/aop/pom.xml                             |   2 +-
 modules/aws/pom.xml                             |   2 +-
 modules/clients/pom.xml                         |   2 +-
 modules/cloud/pom.xml                           |   2 +-
 modules/codegen/pom.xml                         |   2 +-
 modules/core/pom.xml                            |   2 +-
 .../configuration/IgniteReflectionFactory.java  |  81 +++-
 .../processors/cache/GridCacheContext.java      |   2 +-
 .../distributed/dht/GridDhtLocalPartition.java  |   3 +-
 .../distributed/dht/GridDhtLockFuture.java      |   2 +-
 .../dht/atomic/GridDhtAtomicCache.java          |   9 +-
 .../GridDhtPartitionsExchangeFuture.java        |  46 +-
 .../datastructures/DataStructuresProcessor.java |  64 +--
 .../processors/hadoop/HadoopJobInfo.java        |   4 +-
 .../hadoop/counter/HadoopCounterWriter.java     |   5 +-
 .../processors/task/GridTaskProcessor.java      |  23 +-
 .../core/src/main/resources/ignite.properties   |   2 +-
 .../GridTaskFailoverAffinityRunTest.java        | 170 +++++++
 .../CacheReadThroughAtomicRestartSelfTest.java  |  32 ++
 ...heReadThroughLocalAtomicRestartSelfTest.java |  32 ++
 .../CacheReadThroughLocalRestartSelfTest.java   |  32 ++
 ...dThroughReplicatedAtomicRestartSelfTest.java |  32 ++
 ...cheReadThroughReplicatedRestartSelfTest.java |  32 ++
 .../cache/CacheReadThroughRestartSelfTest.java  | 133 ++++++
 .../cache/GridCacheAbstractSelfTest.java        |   2 +-
 ...eDynamicCacheStartNoExchangeTimeoutTest.java | 466 +++++++++++++++++++
 .../cache/IgniteDynamicCacheStartSelfTest.java  |  37 ++
 ...GridCacheQueueMultiNodeAbstractSelfTest.java |   4 +-
 .../GridCacheSetAbstractSelfTest.java           |  22 +-
 .../IgniteDataStructureWithJobTest.java         | 111 +++++
 ...ridCachePartitionNotLoadedEventSelfTest.java |  82 ++++
 .../distributed/IgniteCache150ClientsTest.java  | 189 ++++++++
 ...teCacheClientNodePartitionsExchangeTest.java |   1 +
 .../distributed/IgniteCacheManyClientsTest.java |   2 +
 .../IgniteCacheTxMessageRecoveryTest.java       |   5 +
 ...idCacheNearOnlyMultiNodeFullApiSelfTest.java |   5 -
 ...achePartitionedMultiNodeFullApiSelfTest.java |  49 +-
 .../GridCacheReplicatedFailoverSelfTest.java    |   5 +
 .../IgniteCacheTxStoreSessionTest.java          |   4 +
 .../testframework/junits/GridAbstractTest.java  |   2 +-
 .../IgniteCacheDataStructuresSelfTestSuite.java |   1 +
 .../ignite/testsuites/IgniteCacheTestSuite.java |   4 +-
 .../testsuites/IgniteCacheTestSuite4.java       |   8 +
 .../testsuites/IgniteClientTestSuite.java       |  38 ++
 .../testsuites/IgniteComputeGridTestSuite.java  |   1 +
 .../ignite/util/TestTcpCommunicationSpi.java    |  21 +
 modules/extdata/p2p/pom.xml                     |   2 +-
 modules/extdata/uri/pom.xml                     |   2 +-
 modules/gce/pom.xml                             |   2 +-
 modules/geospatial/pom.xml                      |   2 +-
 modules/hadoop/pom.xml                          |  80 +---
 .../fs/IgniteHadoopFileSystemCounterWriter.java |   9 +-
 .../processors/hadoop/HadoopClassLoader.java    |  29 ++
 .../processors/hadoop/HadoopDefaultJobInfo.java |  27 +-
 .../internal/processors/hadoop/HadoopUtils.java | 237 ----------
 .../hadoop/SecondaryFileSystemProvider.java     |   3 +-
 .../hadoop/fs/HadoopFileSystemCacheUtils.java   | 241 ++++++++++
 .../hadoop/fs/HadoopFileSystemsUtils.java       |  11 +
 .../hadoop/fs/HadoopLazyConcurrentMap.java      |   5 +
 .../hadoop/jobtracker/HadoopJobTracker.java     |  25 +-
 .../child/HadoopChildProcessRunner.java         |   3 +-
 .../processors/hadoop/v2/HadoopV2Job.java       |  84 +++-
 .../hadoop/v2/HadoopV2JobResourceManager.java   |  22 +-
 .../hadoop/v2/HadoopV2TaskContext.java          |  37 +-
 .../apache/ignite/igfs/IgfsEventsTestSuite.java |   5 +-
 .../processors/hadoop/HadoopMapReduceTest.java  |   2 +-
 .../processors/hadoop/HadoopTasksV1Test.java    |   7 +-
 .../processors/hadoop/HadoopTasksV2Test.java    |   7 +-
 .../processors/hadoop/HadoopV2JobSelfTest.java  |   6 +-
 .../collections/HadoopAbstractMapTest.java      |   3 +-
 .../testsuites/IgniteHadoopTestSuite.java       |   2 +-
 .../IgniteIgfsLinuxAndMacOSTestSuite.java       |   3 +-
 modules/hibernate/pom.xml                       |   2 +-
 modules/indexing/pom.xml                        |   2 +-
 ...QueryOffheapEvictsMultiThreadedSelfTest.java |   5 +
 .../IgniteCacheQuerySelfTestSuite.java          |   2 +-
 modules/jcl/pom.xml                             |   2 +-
 modules/jta/pom.xml                             |   2 +-
 modules/log4j/pom.xml                           |   2 +-
 modules/mesos/pom.xml                           |   2 +-
 modules/rest-http/pom.xml                       |   2 +-
 modules/scalar-2.10/pom.xml                     |   2 +-
 modules/scalar/pom.xml                          |   2 +-
 modules/schedule/pom.xml                        |   2 +-
 modules/schema-import/pom.xml                   |   2 +-
 modules/slf4j/pom.xml                           |   2 +-
 modules/spark-2.10/pom.xml                      |   2 +-
 modules/spark/pom.xml                           |   2 +-
 modules/spring/pom.xml                          |   2 +-
 modules/ssh/pom.xml                             |   2 +-
 modules/tools/pom.xml                           |   2 +-
 modules/urideploy/pom.xml                       |   2 +-
 modules/visor-console-2.10/pom.xml              |   2 +-
 modules/visor-console/pom.xml                   |   2 +-
 modules/visor-plugins/pom.xml                   |   2 +-
 modules/web/pom.xml                             |   2 +-
 .../IgniteWebSessionSelfTestSuite.java          |   2 +-
 modules/yardstick/pom.xml                       |   2 +-
 pom.xml                                         |   2 +-
 100 files changed, 2169 insertions(+), 521 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/a9d0662b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/atomic/GridDhtAtomicCache.java
----------------------------------------------------------------------


[12/13] incubator-ignite git commit: IGNITE-621 - Merge branch ignite-621 into sprint-7

Posted by ag...@apache.org.
IGNITE-621 - Merge branch ignite-621 into sprint-7


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

Branch: refs/heads/ignite-sprint-7
Commit: 1427d75b9ea46d3044caddc51b44c03abc71901d
Parents: c866902 ac93958
Author: Alexey Goncharuk <ag...@gridgain.com>
Authored: Thu Jul 2 12:10:09 2015 -0700
Committer: Alexey Goncharuk <ag...@gridgain.com>
Committed: Thu Jul 2 12:10:09 2015 -0700

----------------------------------------------------------------------
 .../java/org/apache/ignite/IgniteCache.java     |   5 +
 .../apache/ignite/IgniteSystemProperties.java   |   3 +
 .../processors/cache/CacheOperationContext.java |  44 ++++-
 .../processors/cache/GridCacheAdapter.java      |  91 ++++++----
 .../processors/cache/GridCacheAtomicFuture.java |  12 +-
 .../processors/cache/GridCacheMvccManager.java  |   8 +-
 .../processors/cache/GridCacheProxyImpl.java    |  10 +-
 .../processors/cache/GridCacheUtils.java        |  42 +++++
 .../processors/cache/IgniteCacheProxy.java      |  36 +++-
 .../dht/atomic/GridDhtAtomicCache.java          |  18 +-
 .../dht/atomic/GridDhtAtomicUpdateFuture.java   |  15 +-
 .../dht/atomic/GridNearAtomicUpdateFuture.java  | 177 +++++++++++++++++--
 .../datastructures/GridCacheAtomicLongImpl.java |  25 +--
 .../GridCacheAtomicSequenceImpl.java            |  11 +-
 .../GridCacheAtomicStampedImpl.java             |  21 +--
 .../GridCacheCountDownLatchImpl.java            |  16 +-
 .../communication/tcp/TcpCommunicationSpi.java  |   2 +-
 .../IgniteCachePutRetryAbstractSelfTest.java    | 147 +++++++++++++++
 .../dht/IgniteCachePutRetryAtomicSelfTest.java  |  34 ++++
 ...gniteCachePutRetryTransactionalSelfTest.java |  74 ++++++++
 ...eAtomicInvalidPartitionHandlingSelfTest.java |   5 +-
 .../IgniteCacheFailoverTestSuite.java           |   3 +
 22 files changed, 665 insertions(+), 134 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1427d75b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/IgniteCacheProxy.java
----------------------------------------------------------------------