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/06/09 00:28:39 UTC

[01/42] incubator-ignite git commit: # GG-9621 WIP.

Repository: incubator-ignite
Updated Branches:
  refs/heads/ignite-389 fa97def49 -> 1552a4b24


# GG-9621 WIP.


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

Branch: refs/heads/ignite-389
Commit: 1497cbe053fe849f2091355f97ba3f4596c719cd
Parents: 59b11b4
Author: AKuznetsov <ak...@gridgain.com>
Authored: Tue Apr 14 18:11:24 2015 +0700
Committer: AKuznetsov <ak...@gridgain.com>
Committed: Tue Apr 14 18:11:24 2015 +0700

----------------------------------------------------------------------
 .../ignite/internal/visor/cache/VisorCache.java  |  2 +-
 .../VisorCacheConfigurationCollectorJob.java     |  6 +++---
 .../internal/visor/cache/VisorCacheMetrics.java  | 19 +++++++++++--------
 .../cache/VisorCacheMetricsCollectorTask.java    | 10 ++++++----
 .../cache/VisorCacheStoreConfiguration.java      |  5 ++---
 5 files changed, 23 insertions(+), 19 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1497cbe0/modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCache.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCache.java b/modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCache.java
index 8c880a7..7de2c2d 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCache.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCache.java
@@ -230,7 +230,7 @@ public class VisorCache implements Serializable {
         cache.partitions = ca.affinity().partitions();
         cache.primaryPartitions = pps;
         cache.backupPartitions = bps;
-        cache.metrics = VisorCacheMetrics.from(ignite, ca);
+        cache.metrics = VisorCacheMetrics.from(ignite, ca.name());
         cache.partitionsMap = partsMap;
 
         return cache;

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1497cbe0/modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCacheConfigurationCollectorJob.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCacheConfigurationCollectorJob.java b/modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCacheConfigurationCollectorJob.java
index ef12424..c8913c6 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCacheConfigurationCollectorJob.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCacheConfigurationCollectorJob.java
@@ -45,17 +45,17 @@ public class VisorCacheConfigurationCollectorJob
 
     /** {@inheritDoc} */
     @Override protected Map<IgniteUuid, VisorCacheConfiguration> run(Collection<IgniteUuid> arg) {
-        Collection<GridCacheAdapter<?, ?>> caches = ignite.context().cache().internalCaches();
+        Collection<IgniteCacheProxy<?, ?>> caches = ignite.context().cache().jcaches();
 
         boolean all = arg == null || arg.isEmpty();
 
         Map<IgniteUuid, VisorCacheConfiguration> res = U.newHashMap(caches.size());
 
-        for (GridCacheAdapter<?, ?> cache : caches) {
+        for (IgniteCacheProxy<?, ?> cache : caches) {
             IgniteUuid deploymentId = cache.context().dynamicDeploymentId();
 
             if (all || arg.contains(deploymentId))
-                res.put(deploymentId, config(cache.configuration()));
+                res.put(deploymentId, config(cache.getConfiguration(CacheConfiguration.class)));
         }
 
         return res;

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1497cbe0/modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCacheMetrics.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCacheMetrics.java b/modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCacheMetrics.java
index b0a1eb2..65a7f1b 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCacheMetrics.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCacheMetrics.java
@@ -17,6 +17,7 @@
 
 package org.apache.ignite.internal.visor.cache;
 
+import org.apache.ignite.*;
 import org.apache.ignite.cache.*;
 import org.apache.ignite.internal.*;
 import org.apache.ignite.internal.processors.cache.*;
@@ -173,19 +174,21 @@ public class VisorCacheMetrics implements Serializable {
 
     /**
      * @param ignite Ignite.
-     * @param c Cache.
+     * @param cacheName Cache name.
      * @return Data transfer object for given cache metrics.
      */
-    public static VisorCacheMetrics from(IgniteEx ignite, GridCache c) {
+    public static VisorCacheMetrics from(IgniteEx ignite, String cacheName) {
         VisorCacheMetrics cm = new VisorCacheMetrics();
 
-        CacheMetrics m = c.metrics();
-
         GridCacheProcessor cacheProcessor = ignite.context().cache();
 
-        cm.name = c.name();
-        cm.mode = cacheProcessor.cacheMode(c.name());
-        cm.sys = cacheProcessor.systemCache(c.name());
+        IgniteCache<Object, Object> c = cacheProcessor.jcache(cacheName);
+
+        cm.name = cacheName;
+        cm.mode = cacheProcessor.cacheMode(cacheName);
+        cm.sys = cacheProcessor.systemCache(cacheName);
+
+        CacheMetrics m = c.metrics();
 
         cm.size = m.getSize();
         cm.keySize = m.getKeySize();
@@ -216,7 +219,7 @@ public class VisorCacheMetrics implements Serializable {
         cm.commitsPerSec = (int)(MICROSECONDS_IN_SECOND * 1.f / m.getAverageTxCommitTime());
         cm.rollbacksPerSec = (int)(MICROSECONDS_IN_SECOND * 1.f / m.getAverageTxRollbackTime());
 
-        cm.qryMetrics = VisorCacheQueryMetrics.from(c.queries().metrics());
+        cm.qryMetrics = VisorCacheQueryMetrics.from(c.queryMetrics());
 
         cm.dhtEvictQueueCurrSize = m.getDhtEvictQueueCurrentSize();
         cm.txThreadMapSize = m.getTxThreadMapSize();

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1497cbe0/modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCacheMetricsCollectorTask.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCacheMetricsCollectorTask.java b/modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCacheMetricsCollectorTask.java
index 8fd42a0..23263c0 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCacheMetricsCollectorTask.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCacheMetricsCollectorTask.java
@@ -99,17 +99,19 @@ public class VisorCacheMetricsCollectorTask extends VisorMultiNodeTask<IgniteBiT
 
             GridCacheProcessor cacheProcessor = ignite.context().cache();
 
-            Collection<GridCacheAdapter<?, ?>> caches = cacheProcessor.internalCaches();
+            Collection<IgniteCacheProxy<?, ?>> caches = cacheProcessor.jcaches();
 
             Collection<VisorCacheMetrics> res = new ArrayList<>(caches.size());
 
             boolean allCaches = cacheNames.isEmpty();
 
-            for (GridCacheAdapter ca : caches) {
+            for (IgniteCacheProxy ca : caches) {
                 if (ca.context().started()) {
-                    VisorCacheMetrics cm = VisorCacheMetrics.from(ignite, ca);
+                    String cacheName = ca.getName();
 
-                    if ((allCaches || cacheNames.contains(ca.name())) && (showSysCaches || !cm.system()))
+                    VisorCacheMetrics cm = VisorCacheMetrics.from(ignite, cacheName);
+
+                    if ((allCaches || cacheNames.contains(cacheName)) && (showSysCaches || !cm.system()))
                         res.add(cm);
                 }
             }

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1497cbe0/modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCacheStoreConfiguration.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCacheStoreConfiguration.java b/modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCacheStoreConfiguration.java
index a4a2235..cacf528 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCacheStoreConfiguration.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCacheStoreConfiguration.java
@@ -17,7 +17,6 @@
 
 package org.apache.ignite.internal.visor.cache;
 
-import org.apache.ignite.*;
 import org.apache.ignite.cache.store.*;
 import org.apache.ignite.cache.store.jdbc.*;
 import org.apache.ignite.configuration.*;
@@ -72,10 +71,10 @@ public class VisorCacheStoreConfiguration implements Serializable {
      * @param ccfg Cache configuration.
      * @return Data transfer object for cache store configuration properties.
      */
-    public static VisorCacheStoreConfiguration from(Ignite ignite, CacheConfiguration ccfg) {
+    public static VisorCacheStoreConfiguration from(IgniteEx ignite, CacheConfiguration ccfg) {
         VisorCacheStoreConfiguration cfg = new VisorCacheStoreConfiguration();
 
-        GridCacheAdapter<Object, Object> c = ((IgniteKernal)ignite).internalCache(ccfg.getName());
+        IgniteCacheProxy<Object, Object> c = ignite.context().cache().jcache(ccfg.getName());
 
         CacheStore store = c != null && c.context().started() ? c.context().store().configuredStore() : null;
 


[19/42] incubator-ignite git commit: Merge remote-tracking branch 'origin/ignite-883_1' into ignite-883_1

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


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

Branch: refs/heads/ignite-389
Commit: a6761529504b9df26703558f9452d53eb9e6dd0f
Parents: ad2f4ef 3417b3d
Author: sboikov <sb...@gridgain.com>
Authored: Fri Jun 5 09:57:06 2015 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Fri Jun 5 09:57:06 2015 +0300

----------------------------------------------------------------------
 .../configuration/CacheConfiguration.java       |  13 +-
 .../apache/ignite/internal/IgniteKernal.java    |   3 +
 .../managers/communication/GridIoManager.java   | 117 ++++----
 .../dht/GridClientPartitionTopology.java        |   2 +-
 .../dht/GridDhtPartitionTopologyImpl.java       |   8 +-
 .../GridDhtPartitionsExchangeFuture.java        |  19 +-
 .../dht/preloader/GridDhtPreloader.java         |   2 +-
 .../processors/hadoop/HadoopTaskContext.java    |  14 +-
 .../igfs/IgfsSecondaryFileSystemImpl.java       |   2 +-
 .../internal/visor/query/VisorQueryJob.java     |   2 +-
 ...niteDynamicCacheWithConfigStartSelfTest.java |  97 +++++++
 .../igfs/IgfsClientCacheSelfTest.java           |   9 +-
 .../IgniteMessagingWithClientTest.java          | 164 +++++++++++
 .../ignite/testsuites/IgniteBasicTestSuite.java |   1 +
 .../testsuites/IgniteCacheTestSuite4.java       |   1 +
 .../fs/IgniteHadoopFileSystemCounterWriter.java |  14 +-
 .../hadoop/fs/v1/IgniteHadoopFileSystem.java    |  70 ++---
 .../hadoop/fs/v2/IgniteHadoopFileSystem.java    |   2 +-
 .../processors/hadoop/HadoopDefaultJobInfo.java |   2 +-
 .../internal/processors/hadoop/HadoopUtils.java | 282 ++++++++++++++++++-
 .../hadoop/SecondaryFileSystemProvider.java     |   4 +-
 .../hadoop/taskexecutor/HadoopRunnableTask.java |  20 +-
 .../processors/hadoop/v2/HadoopV2Job.java       |  31 +-
 .../hadoop/v2/HadoopV2JobResourceManager.java   |  26 +-
 .../hadoop/v2/HadoopV2TaskContext.java          |  48 +++-
 .../hadoop/HadoopClientProtocolSelfTest.java    |   6 +-
 .../hadoop/HadoopAbstractSelfTest.java          |  14 +-
 .../hadoop/HadoopCommandLineTest.java           |  14 +-
 .../processors/hadoop/HadoopMapReduceTest.java  | 176 +++++++++++-
 .../hadoop/HadoopTaskExecutionSelfTest.java     |   2 +-
 .../hadoop/HadoopTasksAllVersionsTest.java      |  15 +-
 .../processors/hadoop/HadoopTasksV1Test.java    |   5 +-
 .../processors/hadoop/HadoopTasksV2Test.java    |   5 +-
 .../processors/hadoop/HadoopV2JobSelfTest.java  |   6 +-
 .../collections/HadoopAbstractMapTest.java      |  12 +
 ...acheConfigurationPrimitiveTypesSelfTest.java | 104 +++++++
 .../IgniteCacheWithIndexingTestSuite.java       |   2 +
 .../commands/cache/VisorCacheScanCommand.scala  |   2 +-
 38 files changed, 1122 insertions(+), 194 deletions(-)
----------------------------------------------------------------------


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


[05/42] incubator-ignite git commit: Merge remote-tracking branch 'remotes/origin/ignite-sprint-5' into ignite-883_1

Posted by ag...@apache.org.
Merge remote-tracking branch 'remotes/origin/ignite-sprint-5' into ignite-883_1


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

Branch: refs/heads/ignite-389
Commit: 7a2e898dd22fd70396d7e7d19b301c0632281347
Parents: 997d65e 4f3788d
Author: sboikov <sb...@gridgain.com>
Authored: Tue Jun 2 12:01:00 2015 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Tue Jun 2 12:01:00 2015 +0300

----------------------------------------------------------------------
 .../hibernate/CacheHibernatePersonStore.java    | 202 +---------
 .../hibernate/CacheHibernateStoreExample.java   |  17 +
 .../store/jdbc/CacheJdbcPersonStore.java        | 180 ++-------
 .../store/jdbc/CacheJdbcStoreExample.java       |  13 +
 .../store/spring/CacheSpringPersonStore.java    | 128 ++++++
 .../store/spring/CacheSpringStoreExample.java   | 143 +++++++
 .../datagrid/store/spring/package-info.java     |  22 ++
 .../apache/ignite/cache/store/CacheStore.java   |   2 +
 .../ignite/cache/store/CacheStoreSession.java   |  22 ++
 .../cache/store/CacheStoreSessionListener.java  | 133 +++++++
 .../jdbc/CacheJdbcStoreSessionListener.java     | 141 +++++++
 .../configuration/CacheConfiguration.java       |  32 ++
 .../configuration/IgniteConfiguration.java      |  38 +-
 .../org/apache/ignite/internal/IgnitionEx.java  |   2 +-
 .../managers/communication/GridIoManager.java   |  12 +-
 .../processors/cache/GridCacheProcessor.java    |  12 +-
 .../cache/GridCacheSharedContext.java           |  47 ++-
 .../processors/cache/GridCacheUtils.java        |  54 +++
 .../cache/store/CacheOsStoreManager.java        |   1 -
 .../cache/store/CacheStoreManager.java          |   7 +-
 .../store/GridCacheStoreManagerAdapter.java     | 202 ++++++++--
 .../cache/transactions/IgniteTxAdapter.java     |  33 +-
 .../transactions/IgniteTxLocalAdapter.java      | 142 ++++---
 ...cheStoreSessionListenerAbstractSelfTest.java | 315 +++++++++++++++
 ...heStoreSessionListenerLifecycleSelfTest.java | 395 +++++++++++++++++++
 .../CacheJdbcStoreSessionListenerSelfTest.java  | 175 ++++++++
 .../IgniteCrossCacheTxStoreSelfTest.java        | 147 ++++---
 .../loadtests/hashmap/GridCacheTestContext.java |   3 +-
 .../junits/cache/TestCacheSession.java          |  18 +
 .../cache/TestThreadLocalCacheSession.java      |  15 +
 .../junits/common/GridCommonAbstractTest.java   |  24 ++
 .../testsuites/IgniteCacheTestSuite4.java       |   3 +
 .../CacheHibernateStoreSessionListener.java     | 216 ++++++++++
 ...heHibernateStoreSessionListenerSelfTest.java | 228 +++++++++++
 .../testsuites/IgniteHibernateTestSuite.java    |   2 +
 .../IgniteCacheQuerySelfTestSuite.java          |   2 +-
 modules/spring/pom.xml                          |  14 +
 .../spring/CacheSpringStoreSessionListener.java | 207 ++++++++++
 ...CacheSpringStoreSessionListenerSelfTest.java | 197 +++++++++
 .../testsuites/IgniteSpringTestSuite.java       |   3 +
 40 files changed, 3057 insertions(+), 492 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/7a2e898d/modules/core/src/main/java/org/apache/ignite/internal/IgnitionEx.java
----------------------------------------------------------------------


[18/42] incubator-ignite git commit: Merge branches 'ignite-gg-9621' and 'ignite-sprint-5' of https://git-wip-us.apache.org/repos/asf/incubator-ignite into ignite-gg-9621

Posted by ag...@apache.org.
Merge branches 'ignite-gg-9621' and 'ignite-sprint-5' of https://git-wip-us.apache.org/repos/asf/incubator-ignite into ignite-gg-9621


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

Branch: refs/heads/ignite-389
Commit: ead66e7ed62f5134b98be984f098655ef4d04c68
Parents: 2b443d9 c9f7291
Author: AKuznetsov <ak...@gridgain.com>
Authored: Fri Jun 5 10:14:08 2015 +0700
Committer: AKuznetsov <ak...@gridgain.com>
Committed: Fri Jun 5 10:14:08 2015 +0700

----------------------------------------------------------------------
 .../managers/communication/GridIoManager.java   | 117 ++++----
 .../dht/GridClientPartitionTopology.java        |   2 +-
 .../dht/GridDhtPartitionTopologyImpl.java       |   8 +-
 .../GridDhtPartitionsExchangeFuture.java        |  19 +-
 .../dht/preloader/GridDhtPreloader.java         |   2 +-
 .../processors/hadoop/HadoopTaskContext.java    |  14 +-
 .../igfs/IgfsSecondaryFileSystemImpl.java       |   2 +-
 ...niteDynamicCacheWithConfigStartSelfTest.java |  97 +++++++
 .../igfs/IgfsClientCacheSelfTest.java           |   9 +-
 .../IgniteMessagingWithClientTest.java          | 164 +++++++++++
 .../ignite/testsuites/IgniteBasicTestSuite.java |   1 +
 .../testsuites/IgniteCacheTestSuite4.java       |   1 +
 .../fs/IgniteHadoopFileSystemCounterWriter.java |  14 +-
 .../hadoop/fs/v1/IgniteHadoopFileSystem.java    |  70 ++---
 .../hadoop/fs/v2/IgniteHadoopFileSystem.java    |   2 +-
 .../processors/hadoop/HadoopDefaultJobInfo.java |   2 +-
 .../internal/processors/hadoop/HadoopUtils.java | 282 ++++++++++++++++++-
 .../hadoop/SecondaryFileSystemProvider.java     |   4 +-
 .../hadoop/taskexecutor/HadoopRunnableTask.java |  20 +-
 .../processors/hadoop/v2/HadoopV2Job.java       |  31 +-
 .../hadoop/v2/HadoopV2JobResourceManager.java   |  26 +-
 .../hadoop/v2/HadoopV2TaskContext.java          |  48 +++-
 .../hadoop/HadoopClientProtocolSelfTest.java    |   6 +-
 .../hadoop/HadoopAbstractSelfTest.java          |  14 +-
 .../hadoop/HadoopCommandLineTest.java           |  14 +-
 .../processors/hadoop/HadoopMapReduceTest.java  | 176 +++++++++++-
 .../hadoop/HadoopTaskExecutionSelfTest.java     |   2 +-
 .../hadoop/HadoopTasksAllVersionsTest.java      |  15 +-
 .../processors/hadoop/HadoopTasksV1Test.java    |   5 +-
 .../processors/hadoop/HadoopTasksV2Test.java    |   5 +-
 .../processors/hadoop/HadoopV2JobSelfTest.java  |   6 +-
 .../collections/HadoopAbstractMapTest.java      |  12 +
 32 files changed, 999 insertions(+), 191 deletions(-)
----------------------------------------------------------------------



[33/42] incubator-ignite git commit: ignite-883

Posted by ag...@apache.org.
ignite-883


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

Branch: refs/heads/ignite-389
Commit: c7bc598a32c06cd1acc6f3e1b162d059d80a53f2
Parents: f4da46c
Author: sboikov <sb...@gridgain.com>
Authored: Mon Jun 8 11:16:57 2015 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Mon Jun 8 11:16:57 2015 +0300

----------------------------------------------------------------------
 .../ignite/internal/managers/discovery/GridDiscoveryManager.java   | 2 +-
 1 file changed, 1 insertion(+), 1 deletion(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/c7bc598a/modules/core/src/main/java/org/apache/ignite/internal/managers/discovery/GridDiscoveryManager.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/managers/discovery/GridDiscoveryManager.java b/modules/core/src/main/java/org/apache/ignite/internal/managers/discovery/GridDiscoveryManager.java
index 5983553..142dbaa 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/managers/discovery/GridDiscoveryManager.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/managers/discovery/GridDiscoveryManager.java
@@ -173,7 +173,7 @@ public class GridDiscoveryManager extends GridManagerAdapter<DiscoverySpi> {
         new ConcurrentHashMap8<>();
 
     /** Map of dynamic cache filters. */
-    private Map<String, CachePredicate> registeredCaches = new ConcurrentHashMap<>();
+    private Map<String, CachePredicate> registeredCaches = new HashMap<>();
 
     /** */
     private final GridSpinBusyLock busyLock = new GridSpinBusyLock();


[13/42] incubator-ignite git commit: Merge branches 'ignite-gg-9621' and 'ignite-sprint-5'

Posted by ag...@apache.org.
Merge branches 'ignite-gg-9621' and 'ignite-sprint-5'


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

Branch: refs/heads/ignite-389
Commit: 2b443d9096c4bc434bd99b2c675bbcbf7e65f236
Parents: 1f516a2
Author: AKuznetsov <ak...@gridgain.com>
Authored: Thu Jun 4 17:12:15 2015 +0700
Committer: AKuznetsov <ak...@gridgain.com>
Committed: Thu Jun 4 17:12:15 2015 +0700

----------------------------------------------------------------------
 .../java/org/apache/ignite/internal/visor/cache/VisorCache.java    | 2 +-
 1 file changed, 1 insertion(+), 1 deletion(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/2b443d90/modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCache.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCache.java b/modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCache.java
index bd24ecf..9e15d2d 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCache.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCache.java
@@ -202,7 +202,7 @@ public class VisorCache implements Serializable {
         offHeapAllocatedSize = ca.offHeapAllocatedSize();
         offHeapEntriesCnt = ca.offHeapEntriesCount();
         partitions = ca.affinity().partitions();
-        metrics = VisorCacheMetrics.from(ignite, ca);
+        metrics = VisorCacheMetrics.from(ignite, cacheName);
 
         estimateMemorySize(ignite, ca, sample);
 


[40/42] incubator-ignite git commit: Merge remote-tracking branch 'origin/ignite-sprint-5' into ignite-sprint-5

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


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

Branch: refs/heads/ignite-389
Commit: ff7827e479b1fbcb79d934d8f240ae0a2ff9fc3d
Parents: f1cfd29 2a8e2ab
Author: avinogradov <av...@gridgain.com>
Authored: Mon Jun 8 15:18:21 2015 +0300
Committer: avinogradov <av...@gridgain.com>
Committed: Mon Jun 8 15:18:21 2015 +0300

----------------------------------------------------------------------
 .../DataStreamerMultinodeCreateCacheTest.java   | 97 ++++++++++++++++++++
 .../ignite/testsuites/IgniteCacheTestSuite.java |  1 +
 2 files changed, 98 insertions(+)
----------------------------------------------------------------------



[23/42] incubator-ignite git commit: # ignite-883

Posted by ag...@apache.org.
# ignite-883


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

Branch: refs/heads/ignite-389
Commit: fb827a7784614343ae639ea8b856d2f9f88d46db
Parents: db57652
Author: sboikov <sb...@gridgain.com>
Authored: Fri Jun 5 11:41:46 2015 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Fri Jun 5 15:10:00 2015 +0300

----------------------------------------------------------------------
 .../datastructures/DataStructuresProcessor.java |  31 +-
 .../timeout/GridSpiTimeoutObject.java           |  14 +
 .../util/nio/GridCommunicationClient.java       |   6 -
 .../util/nio/GridTcpCommunicationClient.java    | 554 -------------------
 .../util/nio/GridTcpNioCommunicationClient.java |   8 -
 .../org/apache/ignite/spi/IgniteSpiAdapter.java |   4 +
 .../communication/tcp/TcpCommunicationSpi.java  |  97 +---
 .../tcp/TcpCommunicationSpiMBean.java           |   2 -
 .../internal/util/nio/GridNioSelfTest.java      |   2 +-
 .../GridTcpCommunicationSpiAbstractTest.java    |   4 +-
 ...mmunicationSpiConcurrentConnectSelfTest.java |   2 +-
 .../GridTcpCommunicationSpiConfigSelfTest.java  |   2 -
 ...cpCommunicationSpiMultithreadedSelfTest.java |   2 +-
 .../discovery/AbstractDiscoverySelfTest.java    |  13 +-
 14 files changed, 61 insertions(+), 680 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/fb827a77/modules/core/src/main/java/org/apache/ignite/internal/processors/datastructures/DataStructuresProcessor.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/datastructures/DataStructuresProcessor.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/datastructures/DataStructuresProcessor.java
index 2138639..aa3bfe2 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/datastructures/DataStructuresProcessor.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/datastructures/DataStructuresProcessor.java
@@ -101,7 +101,7 @@ public final class DataStructuresProcessor extends GridProcessorAdapter {
     private IgniteInternalCache<CacheDataStructuresCacheKey, List<CacheCollectionInfo>> utilityDataCache;
 
     /** */
-    private UUID qryId;
+    private volatile UUID qryId;
 
     /**
      * @param ctx Context.
@@ -144,11 +144,22 @@ public final class DataStructuresProcessor extends GridProcessorAdapter {
             seqView = atomicsCache;
 
             dsCacheCtx = atomicsCache.context();
+        }
+    }
 
-            qryId = dsCacheCtx.continuousQueries().executeInternalQuery(new DataStructuresEntryListener(),
-                new DataStructuresEntryFilter(),
-                dsCacheCtx.isReplicated() && dsCacheCtx.affinityNode(),
-                false);
+    /**
+     * @throws IgniteCheckedException If failed.
+     */
+    private void startQuery() throws IgniteCheckedException {
+        if (qryId == null) {
+            synchronized (this) {
+                if (qryId == null) {
+                    qryId = dsCacheCtx.continuousQueries().executeInternalQuery(new DataStructuresEntryListener(),
+                        new DataStructuresEntryFilter(),
+                        dsCacheCtx.isReplicated() && dsCacheCtx.affinityNode(),
+                        false);
+                }
+            }
         }
     }
 
@@ -178,6 +189,8 @@ public final class DataStructuresProcessor extends GridProcessorAdapter {
 
         checkAtomicsConfiguration();
 
+        startQuery();
+
         return getAtomic(new IgniteOutClosureX<IgniteAtomicSequence>() {
             @Override public IgniteAtomicSequence applyx() throws IgniteCheckedException {
                 GridCacheInternalKey key = new GridCacheInternalKeyImpl(name);
@@ -304,6 +317,8 @@ public final class DataStructuresProcessor extends GridProcessorAdapter {
 
         checkAtomicsConfiguration();
 
+        startQuery();
+
         return getAtomic(new IgniteOutClosureX<IgniteAtomicLong>() {
             @Override public IgniteAtomicLong applyx() throws IgniteCheckedException {
                 final GridCacheInternalKey key = new GridCacheInternalKeyImpl(name);
@@ -507,6 +522,8 @@ public final class DataStructuresProcessor extends GridProcessorAdapter {
 
         checkAtomicsConfiguration();
 
+        startQuery();
+
         return getAtomic(new IgniteOutClosureX<IgniteAtomicReference>() {
             @Override public IgniteAtomicReference<T> applyx() throws IgniteCheckedException {
                 GridCacheInternalKey key = new GridCacheInternalKeyImpl(name);
@@ -608,6 +625,8 @@ public final class DataStructuresProcessor extends GridProcessorAdapter {
 
         checkAtomicsConfiguration();
 
+        startQuery();
+
         return getAtomic(new IgniteOutClosureX<IgniteAtomicStamped>() {
             @Override public IgniteAtomicStamped<T, S> applyx() throws IgniteCheckedException {
                 GridCacheInternalKeyImpl key = new GridCacheInternalKeyImpl(name);
@@ -916,6 +935,8 @@ public final class DataStructuresProcessor extends GridProcessorAdapter {
 
         checkAtomicsConfiguration();
 
+        startQuery();
+
         return getAtomic(new IgniteOutClosureX<IgniteCountDownLatch>() {
             @Override public IgniteCountDownLatch applyx() throws IgniteCheckedException {
                 GridCacheInternalKey key = new GridCacheInternalKeyImpl(name);

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/fb827a77/modules/core/src/main/java/org/apache/ignite/internal/processors/timeout/GridSpiTimeoutObject.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/timeout/GridSpiTimeoutObject.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/timeout/GridSpiTimeoutObject.java
index 82267a2..a0fd9b4 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/timeout/GridSpiTimeoutObject.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/timeout/GridSpiTimeoutObject.java
@@ -53,6 +53,20 @@ public class GridSpiTimeoutObject implements GridTimeoutObject {
     }
 
     /** {@inheritDoc} */
+    @Override public int hashCode() {
+        assert false;
+
+        return super.hashCode();
+    }
+
+    /** {@inheritDoc} */
+    @Override public boolean equals(Object obj) {
+        assert false;
+
+        return super.equals(obj);
+    }
+
+    /** {@inheritDoc} */
     @Override public final String toString() {
         return S.toString(GridSpiTimeoutObject.class, this);
     }

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/fb827a77/modules/core/src/main/java/org/apache/ignite/internal/util/nio/GridCommunicationClient.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/util/nio/GridCommunicationClient.java b/modules/core/src/main/java/org/apache/ignite/internal/util/nio/GridCommunicationClient.java
index 2f7fd88..693a5a4 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/util/nio/GridCommunicationClient.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/util/nio/GridCommunicationClient.java
@@ -100,12 +100,6 @@ public interface GridCommunicationClient {
     public boolean sendMessage(@Nullable UUID nodeId, Message msg) throws IgniteCheckedException;
 
     /**
-     * @param timeout Timeout.
-     * @throws IOException If failed.
-     */
-    public void flushIfNeeded(long timeout) throws IOException;
-
-    /**
      * @return {@code True} if send is asynchronous.
      */
     public boolean async();

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/fb827a77/modules/core/src/main/java/org/apache/ignite/internal/util/nio/GridTcpCommunicationClient.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/util/nio/GridTcpCommunicationClient.java b/modules/core/src/main/java/org/apache/ignite/internal/util/nio/GridTcpCommunicationClient.java
deleted file mode 100644
index 72c20f8..0000000
--- a/modules/core/src/main/java/org/apache/ignite/internal/util/nio/GridTcpCommunicationClient.java
+++ /dev/null
@@ -1,554 +0,0 @@
-/*
- * 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.util.nio;
-
-import org.apache.ignite.*;
-import org.apache.ignite.internal.util.lang.*;
-import org.apache.ignite.internal.util.typedef.internal.*;
-import org.apache.ignite.plugin.extensions.communication.*;
-import org.jetbrains.annotations.*;
-
-import java.io.*;
-import java.net.*;
-import java.nio.*;
-import java.util.*;
-import java.util.concurrent.locks.*;
-
-/**
- * Grid client for NIO server.
- */
-public class GridTcpCommunicationClient extends GridAbstractCommunicationClient {
-    /** Socket. */
-    private final Socket sock;
-
-    /** Output stream. */
-    private final UnsafeBufferedOutputStream out;
-
-    /** Minimum buffered message count. */
-    private final int minBufferedMsgCnt;
-
-    /** Communication buffer size ratio. */
-    private final double bufSizeRatio;
-
-    /** */
-    private final ByteBuffer writeBuf;
-
-    /** */
-    private final MessageFormatter formatter;
-
-    /**
-     * @param metricsLsnr Metrics listener.
-     * @param addr Address.
-     * @param locHost Local address.
-     * @param connTimeout Connect timeout.
-     * @param tcpNoDelay Value for {@code TCP_NODELAY} socket option.
-     * @param sockRcvBuf Socket receive buffer.
-     * @param sockSndBuf Socket send buffer.
-     * @param bufSize Buffer size (or {@code 0} to disable buffer).
-     * @param minBufferedMsgCnt Minimum buffered message count.
-     * @param bufSizeRatio Communication buffer size ratio.
-     * @param formatter Message formatter.
-     * @throws IgniteCheckedException If failed.
-     */
-    public GridTcpCommunicationClient(
-        GridNioMetricsListener metricsLsnr,
-        InetSocketAddress addr,
-        InetAddress locHost,
-        long connTimeout,
-        boolean tcpNoDelay,
-        int sockRcvBuf,
-        int sockSndBuf,
-        int bufSize,
-        int minBufferedMsgCnt,
-        double bufSizeRatio,
-        MessageFormatter formatter
-    ) throws IgniteCheckedException {
-        super(metricsLsnr);
-
-        assert metricsLsnr != null;
-        assert addr != null;
-        assert locHost != null;
-        assert connTimeout >= 0;
-        assert bufSize >= 0;
-
-        A.ensure(minBufferedMsgCnt >= 0,
-            "Value of minBufferedMessageCount property cannot be less than zero.");
-        A.ensure(bufSizeRatio > 0 && bufSizeRatio < 1,
-            "Value of bufSizeRatio property must be between 0 and 1 (exclusive).");
-
-        this.minBufferedMsgCnt = minBufferedMsgCnt;
-        this.bufSizeRatio = bufSizeRatio;
-        this.formatter = formatter;
-
-        writeBuf = ByteBuffer.allocate(8 << 10);
-
-        writeBuf.order(ByteOrder.nativeOrder());
-
-        sock = new Socket();
-
-        boolean success = false;
-
-        try {
-            sock.bind(new InetSocketAddress(locHost, 0));
-
-            sock.setTcpNoDelay(tcpNoDelay);
-
-            if (sockRcvBuf > 0)
-                sock.setReceiveBufferSize(sockRcvBuf);
-
-            if (sockSndBuf > 0)
-                sock.setSendBufferSize(sockSndBuf);
-
-            sock.connect(addr, (int)connTimeout);
-
-            out = new UnsafeBufferedOutputStream(sock.getOutputStream(), bufSize);
-
-            success = true;
-        }
-        catch (IOException e) {
-            throw new IgniteCheckedException("Failed to connect to remote host " +
-                "[addr=" + addr + ", localHost=" + locHost + ']', e);
-        }
-        finally {
-            if (!success)
-                U.closeQuiet(sock);
-        }
-    }
-
-    /** {@inheritDoc} */
-    @Override public void doHandshake(IgniteInClosure2X<InputStream, OutputStream> handshakeC) throws IgniteCheckedException {
-        try {
-            handshakeC.applyx(sock.getInputStream(), sock.getOutputStream());
-        }
-        catch (IOException e) {
-            throw new IgniteCheckedException("Failed to access IO streams when executing handshake with remote node: " +
-                sock.getRemoteSocketAddress(), e);
-        }
-    }
-
-    /** {@inheritDoc} */
-    @Override public boolean close() {
-        boolean res = super.close();
-
-        if (res) {
-            U.closeQuiet(out);
-            U.closeQuiet(sock);
-        }
-
-        return res;
-    }
-
-    /** {@inheritDoc} */
-    @Override public void forceClose() {
-        super.forceClose();
-
-        try {
-            out.flush();
-        }
-        catch (IOException ignored) {
-            // No-op.
-        }
-
-        // Do not call (directly or indirectly) out.close() here
-        // since it may cause a deadlock.
-        out.forceClose();
-
-        U.closeQuiet(sock);
-    }
-
-    /** {@inheritDoc} */
-    @Override public void sendMessage(byte[] data, int len) throws IgniteCheckedException {
-        if (closed())
-            throw new IgniteCheckedException("Client was closed: " + this);
-
-        try {
-            out.write(data, 0, len);
-
-            metricsLsnr.onBytesSent(len);
-        }
-        catch (IOException e) {
-            throw new IgniteCheckedException("Failed to send message to remote node: " + sock.getRemoteSocketAddress(), e);
-        }
-
-        markUsed();
-    }
-
-    /** {@inheritDoc} */
-    @Override public boolean sendMessage(@Nullable UUID nodeId, Message msg)
-        throws IgniteCheckedException {
-        if (closed())
-            throw new IgniteCheckedException("Client was closed: " + this);
-
-        assert writeBuf.hasArray();
-
-        try {
-            int cnt = U.writeMessageFully(msg, out, writeBuf, formatter.writer());
-
-            metricsLsnr.onBytesSent(cnt);
-        }
-        catch (IOException e) {
-            throw new IgniteCheckedException("Failed to send message to remote node: " + sock.getRemoteSocketAddress(), e);
-        }
-
-        markUsed();
-
-        return false;
-    }
-
-    /**
-     * @param timeout Timeout.
-     * @throws IOException If failed.
-     */
-    @Override public void flushIfNeeded(long timeout) throws IOException {
-        assert timeout > 0;
-
-        out.flushOnTimeout(timeout);
-    }
-
-    /** {@inheritDoc} */
-    @Override public void sendMessage(ByteBuffer data) throws IgniteCheckedException {
-        throw new UnsupportedOperationException();
-    }
-
-    /** {@inheritDoc} */
-    @Override public String toString() {
-        return S.toString(GridTcpCommunicationClient.class, this, super.toString());
-    }
-
-    /**
-     *
-     */
-    private class UnsafeBufferedOutputStream extends FilterOutputStream {
-        /** The internal buffer where data is stored. */
-        private final byte buf[];
-
-        /** Current size. */
-        private int size;
-
-        /** Count. */
-        private int cnt;
-
-        /** Message count. */
-        private int msgCnt;
-
-        /** Total messages size. */
-        private int totalCnt;
-
-        /** Lock. */
-        private final ReentrantLock lock = new ReentrantLock();
-
-        /** Last flushed timestamp. */
-        private volatile long lastFlushed = U.currentTimeMillis();
-
-        /** Cached flush timeout. */
-        private volatile long flushTimeout;
-
-        /** Buffer adjusted timestamp. */
-        private long lastAdjusted = U.currentTimeMillis();
-
-        /**
-         * Creates a new buffered output stream to write data to the
-         * specified underlying output stream.
-         *
-         * @param out The underlying output stream.
-         */
-        UnsafeBufferedOutputStream(OutputStream out) {
-            this(out, 8192);
-        }
-
-        /**
-         * Creates a new buffered output stream to write data to the
-         * specified underlying output stream with the specified buffer
-         * size.
-         *
-         * @param out The underlying output stream.
-         * @param size The buffer size.
-         */
-        UnsafeBufferedOutputStream(OutputStream out, int size) {
-            super(out);
-
-            assert size >= 0;
-
-            this.size = size;
-            buf = size > 0 ? new byte[size] : null;
-        }
-
-        /** {@inheritDoc} */
-        @Override public void write(int b) throws IOException {
-            throw new UnsupportedOperationException();
-        }
-
-        /** {@inheritDoc} */
-        @Override public void write(byte[] b, int off, int len) throws IOException {
-            assert b != null;
-            assert off == 0;
-
-            // No buffering.
-            if (buf == null) {
-                lock.lock();
-
-                try {
-                    out.write(b, 0, len);
-                }
-                finally {
-                    lock.unlock();
-                }
-
-                return;
-            }
-
-            // Buffering is enabled.
-            lock.lock();
-
-            try {
-                msgCnt++;
-                totalCnt += len;
-
-                if (len >= size) {
-                    flushLocked();
-
-                    out.write(b, 0, len);
-
-                    lastFlushed = U.currentTimeMillis();
-
-                    adjustBufferIfNeeded();
-
-                    return;
-                }
-
-                if (cnt + len > size) {
-                    flushLocked();
-
-                    messageToBuffer0(b, off, len, buf, 0);
-
-                    cnt = len;
-
-                    assert cnt < size;
-
-                    adjustBufferIfNeeded();
-
-                    return;
-                }
-
-                messageToBuffer0(b, 0, len, buf, cnt);
-
-                cnt += len;
-
-                if (cnt == size)
-                    flushLocked();
-                else
-                    flushIfNeeded();
-            }
-            finally {
-                lock.unlock();
-            }
-        }
-
-        /**
-         * @throws IOException If failed.
-         */
-        private void flushIfNeeded() throws IOException {
-            assert lock.isHeldByCurrentThread();
-            assert buf != null;
-
-            long flushTimeout0 = flushTimeout;
-
-            if (flushTimeout0 > 0)
-                flushOnTimeoutLocked(flushTimeout0);
-        }
-
-        /**
-         *
-         */
-        private void adjustBufferIfNeeded() {
-            assert lock.isHeldByCurrentThread();
-            assert buf != null;
-
-            long flushTimeout0 = flushTimeout;
-
-            if (flushTimeout0 > 0)
-                adjustBufferLocked(flushTimeout0);
-        }
-
-        /** {@inheritDoc} */
-        @Override public void flush() throws IOException {
-            lock.lock();
-
-            try {
-                flushLocked();
-            }
-            finally {
-                lock.unlock();
-            }
-        }
-
-        /**
-         * @param timeout Timeout.
-         * @throws IOException If failed.
-         */
-        public void flushOnTimeout(long timeout) throws IOException {
-            assert buf != null;
-            assert timeout > 0;
-
-            // Overwrite cached value.
-            flushTimeout = timeout;
-
-            if (lastFlushed + timeout > U.currentTimeMillis() || !lock.tryLock())
-                return;
-
-            try {
-                flushOnTimeoutLocked(timeout);
-            }
-            finally {
-                lock.unlock();
-            }
-        }
-
-        /**
-         * @param timeout Timeout.
-         * @throws IOException If failed.
-         */
-        private void flushOnTimeoutLocked(long timeout) throws IOException {
-            assert lock.isHeldByCurrentThread();
-            assert timeout > 0;
-
-            // Double check.
-            if (cnt == 0 || lastFlushed + timeout > U.currentTimeMillis())
-                return;
-
-            flushLocked();
-
-            adjustBufferLocked(timeout);
-        }
-
-        /**
-         * @param timeout Timeout.
-         */
-        private void adjustBufferLocked(long timeout) {
-            assert lock.isHeldByCurrentThread();
-            assert timeout > 0;
-
-            long time = U.currentTimeMillis();
-
-            if (lastAdjusted + timeout < time) {
-                if (msgCnt <= minBufferedMsgCnt)
-                    size = 0;
-                else {
-                    size = (int)(totalCnt * bufSizeRatio);
-
-                    if (size > buf.length)
-                        size = buf.length;
-                }
-
-                msgCnt = 0;
-                totalCnt = 0;
-
-                lastAdjusted = time;
-            }
-        }
-
-        /**
-         * @throws IOException If failed.
-         */
-        private void flushLocked() throws IOException {
-            assert lock.isHeldByCurrentThread();
-
-            if (buf != null && cnt > 0) {
-                out.write(buf, 0, cnt);
-
-                cnt = 0;
-            }
-
-            out.flush();
-
-            lastFlushed = U.currentTimeMillis();
-        }
-
-        /** {@inheritDoc} */
-        @Override public void close() throws IOException {
-            lock.lock();
-
-            try {
-                flushLocked();
-            }
-            finally {
-                try {
-                    out.close();
-                }
-                finally {
-                    lock.unlock();
-                }
-            }
-        }
-
-        /**
-         * Forcibly closes underlying stream ignoring any possible exception.
-         */
-        public void forceClose() {
-            try {
-                out.close();
-            }
-            catch (IOException ignored) {
-                // No-op.
-            }
-        }
-
-        /**
-         * @param b Buffer to copy from.
-         * @param off Offset in source buffer.
-         * @param len Length.
-         * @param resBuf Result buffer.
-         * @param resOff Result offset.
-         */
-        private void messageToBuffer(byte[] b, int off, int len, byte[] resBuf, int resOff) {
-            assert b.length == len;
-            assert off == 0;
-            assert resBuf.length >= resOff + len + 4;
-
-            U.intToBytes(len, resBuf, resOff);
-
-            U.arrayCopy(b, off, resBuf, resOff + 4, len);
-        }
-
-        /**
-         * @param b Buffer to copy from (length included).
-         * @param off Offset in source buffer.
-         * @param len Length.
-         * @param resBuf Result buffer.
-         * @param resOff Result offset.
-         */
-        private void messageToBuffer0(byte[] b, int off, int len, byte[] resBuf, int resOff) {
-            assert off == 0;
-            assert resBuf.length >= resOff + len;
-
-            U.arrayCopy(b, off, resBuf, resOff, len);
-        }
-
-        /** {@inheritDoc} */
-        @Override public String toString() {
-            lock.lock();
-
-            try {
-                return S.toString(UnsafeBufferedOutputStream.class, this);
-            }
-            finally {
-                lock.unlock();
-            }
-        }
-    }
-}

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/fb827a77/modules/core/src/main/java/org/apache/ignite/internal/util/nio/GridTcpNioCommunicationClient.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/util/nio/GridTcpNioCommunicationClient.java b/modules/core/src/main/java/org/apache/ignite/internal/util/nio/GridTcpNioCommunicationClient.java
index 788a8e6..abad875 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/util/nio/GridTcpNioCommunicationClient.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/util/nio/GridTcpNioCommunicationClient.java
@@ -122,14 +122,6 @@ public class GridTcpNioCommunicationClient extends GridAbstractCommunicationClie
         return false;
     }
 
-    /**
-     * @param timeout Timeout.
-     * @throws IOException If failed.
-     */
-    @Override public void flushIfNeeded(long timeout) throws IOException {
-        // No-op.
-    }
-
     /** {@inheritDoc} */
     @Override public boolean async() {
         return true;

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/fb827a77/modules/core/src/main/java/org/apache/ignite/spi/IgniteSpiAdapter.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/spi/IgniteSpiAdapter.java b/modules/core/src/main/java/org/apache/ignite/spi/IgniteSpiAdapter.java
index c9c633f..d095491 100644
--- a/modules/core/src/main/java/org/apache/ignite/spi/IgniteSpiAdapter.java
+++ b/modules/core/src/main/java/org/apache/ignite/spi/IgniteSpiAdapter.java
@@ -730,11 +730,15 @@ public abstract class IgniteSpiAdapter implements IgniteSpi, IgniteSpiManagement
 
         /** {@inheritDoc} */
         @Override public void addTimeoutObject(IgniteSpiTimeoutObject obj) {
+            assert ignite instanceof IgniteKernal : ignite;
+
             ((IgniteKernal)ignite).context().timeout().addTimeoutObject(new GridSpiTimeoutObject(obj));
         }
 
         /** {@inheritDoc} */
         @Override public void removeTimeoutObject(IgniteSpiTimeoutObject obj) {
+            assert ignite instanceof IgniteKernal : ignite;
+
             ((IgniteKernal)ignite).context().timeout().removeTimeoutObject(new GridSpiTimeoutObject(obj));
         }
     }

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/fb827a77/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 b324ab2..359de1c 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
@@ -157,12 +157,6 @@ public class TcpCommunicationSpi extends IgniteSpiAdapter
     /** Default idle connection timeout (value is <tt>30000</tt>ms). */
     public static final long DFLT_IDLE_CONN_TIMEOUT = 30000;
 
-    /** Default value for connection buffer flush frequency (value is <tt>100</tt> ms). */
-    public static final long DFLT_CONN_BUF_FLUSH_FREQ = 100;
-
-    /** Default value for connection buffer size (value is <tt>0</tt>). */
-    public static final int DFLT_CONN_BUF_SIZE = 0;
-
     /** Default socket send and receive buffer size. */
     public static final int DFLT_SOCK_BUF_SIZE = 32 * 1024;
 
@@ -603,13 +597,6 @@ public class TcpCommunicationSpi extends IgniteSpiAdapter
     /** Idle connection timeout. */
     private long idleConnTimeout = DFLT_IDLE_CONN_TIMEOUT;
 
-    /** Connection buffer flush frequency. */
-    private volatile long connBufFlushFreq = DFLT_CONN_BUF_FLUSH_FREQ;
-
-    /** Connection buffer size. */
-    @SuppressWarnings("RedundantFieldInitialization")
-    private int connBufSize = DFLT_CONN_BUF_SIZE;
-
     /** Connect timeout. */
     private long connTimeout = DFLT_CONN_TIMEOUT;
 
@@ -647,9 +634,6 @@ public class TcpCommunicationSpi extends IgniteSpiAdapter
     /** Socket write timeout. */
     private long sockWriteTimeout = DFLT_SOCK_WRITE_TIMEOUT;
 
-    /** Flush client worker. */
-    private ClientFlushWorker clientFlushWorker;
-
     /** Recovery and idle clients handler. */
     private CommunicationWorker commWorker;
 
@@ -876,31 +860,29 @@ public class TcpCommunicationSpi extends IgniteSpiAdapter
 
     /**
      * Sets connection buffer size. If set to {@code 0} connection buffer is disabled.
-     * <p>
-     * If not provided, default value is {@link #DFLT_CONN_BUF_SIZE}.
      *
      * @param connBufSize Connection buffer size.
      * @see #setConnectionBufferFlushFrequency(long)
      */
     @IgniteSpiConfiguration(optional = true)
     public void setConnectionBufferSize(int connBufSize) {
-        this.connBufSize = connBufSize;
+        // No-op.
     }
 
     /** {@inheritDoc} */
     @Override public int getConnectionBufferSize() {
-        return connBufSize;
+        return 0;
     }
 
     /** {@inheritDoc} */
     @IgniteSpiConfiguration(optional = true)
     @Override public void setConnectionBufferFlushFrequency(long connBufFlushFreq) {
-        this.connBufFlushFreq = connBufFlushFreq;
+        // No-op.
     }
 
     /** {@inheritDoc} */
     @Override public long getConnectionBufferFlushFrequency() {
-        return connBufFlushFreq;
+        return 0;
     }
 
     /**
@@ -1168,8 +1150,6 @@ public class TcpCommunicationSpi extends IgniteSpiAdapter
         assertParameter(locPort <= 0xffff, "locPort < 0xffff");
         assertParameter(locPortRange >= 0, "locPortRange >= 0");
         assertParameter(idleConnTimeout > 0, "idleConnTimeout > 0");
-        assertParameter(connBufFlushFreq > 0, "connBufFlushFreq > 0");
-        assertParameter(connBufSize >= 0, "connBufSize >= 0");
         assertParameter(sockRcvBuf >= 0, "sockRcvBuf >= 0");
         assertParameter(sockSndBuf >= 0, "sockSndBuf >= 0");
         assertParameter(msgQueueLimit >= 0, "msgQueueLimit >= 0");
@@ -1239,8 +1219,6 @@ public class TcpCommunicationSpi extends IgniteSpiAdapter
             log.debug(configInfo("idleConnTimeout", idleConnTimeout));
             log.debug(configInfo("directBuf", directBuf));
             log.debug(configInfo("directSendBuf", directSndBuf));
-            log.debug(configInfo("connBufSize", connBufSize));
-            log.debug(configInfo("connBufFlushFreq", connBufFlushFreq));
             log.debug(configInfo("selectorsCnt", selectorsCnt));
             log.debug(configInfo("tcpNoDelay", tcpNoDelay));
             log.debug(configInfo("sockSndBuf", sockSndBuf));
@@ -1255,11 +1233,6 @@ public class TcpCommunicationSpi extends IgniteSpiAdapter
             log.debug(configInfo("unackedMsgsBufSize", unackedMsgsBufSize));
         }
 
-        if (connBufSize > 8192)
-            U.warn(log, "Specified communication IO buffer size is larger than recommended (ignore if done " +
-                "intentionally) [specified=" + connBufSize + ", recommended=8192]",
-                "Specified communication IO buffer size is larger than recommended (ignore if done intentionally).");
-
         if (!tcpNoDelay)
             U.quietAndWarn(log, "'TCP_NO_DELAY' for communication is off, which should be used with caution " +
                 "since may produce significant delays with some scenarios.");
@@ -1272,12 +1245,6 @@ public class TcpCommunicationSpi extends IgniteSpiAdapter
 
         commWorker.start();
 
-        if (connBufSize > 0) {
-            clientFlushWorker = new ClientFlushWorker();
-
-            clientFlushWorker.start();
-        }
-
         // Ack start.
         if (log.isDebugEnabled())
             log.debug(startInfo());
@@ -1431,10 +1398,8 @@ public class TcpCommunicationSpi extends IgniteSpiAdapter
         if (nioSrvr != null)
             nioSrvr.stop();
 
-        U.interrupt(clientFlushWorker);
         U.interrupt(commWorker);
 
-        U.join(clientFlushWorker, log);
         U.join(commWorker, log);
 
         // Force closing on stop (safety).
@@ -2023,10 +1988,8 @@ public class TcpCommunicationSpi extends IgniteSpiAdapter
         if (nioSrvr != null)
             nioSrvr.stop();
 
-        U.interrupt(clientFlushWorker);
         U.interrupt(commWorker);
 
-        U.join(clientFlushWorker, log);
         U.join(commWorker, log);
 
         for (GridCommunicationClient client : clients.values())
@@ -2134,58 +2097,6 @@ public class TcpCommunicationSpi extends IgniteSpiAdapter
     /**
      *
      */
-    private class ClientFlushWorker extends IgniteSpiThread {
-        /**
-         *
-         */
-        ClientFlushWorker() {
-            super(gridName, "nio-client-flusher", log);
-        }
-
-        /** {@inheritDoc} */
-        @SuppressWarnings({"BusyWait"})
-        @Override protected void body() throws InterruptedException {
-            while (!isInterrupted()) {
-                long connBufFlushFreq0 = connBufFlushFreq;
-
-                for (Map.Entry<UUID, GridCommunicationClient> entry : clients.entrySet()) {
-                    GridCommunicationClient client = entry.getValue();
-
-                    if (client.reserve()) {
-                        boolean err = true;
-
-                        try {
-                            client.flushIfNeeded(connBufFlushFreq0);
-
-                            err = false;
-                        }
-                        catch (IOException e) {
-                            if (getSpiContext().pingNode(entry.getKey()))
-                                U.error(log, "Failed to flush client: " + client, e);
-                            else {
-                                if (log.isDebugEnabled())
-                                    log.debug("Failed to flush client (node left): " + client);
-
-                                onException("Failed to flush client (node left): " + client, e);
-                            }
-                        }
-                        finally {
-                            if (err)
-                                client.forceClose();
-                            else
-                                client.release();
-                        }
-                    }
-                }
-
-                Thread.sleep(connBufFlushFreq0);
-            }
-        }
-    }
-
-    /**
-     *
-     */
     private class CommunicationWorker extends IgniteSpiThread {
         /** */
         private final BlockingQueue<GridNioRecoveryDescriptor> q = new LinkedBlockingQueue<>();

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/fb827a77/modules/core/src/main/java/org/apache/ignite/spi/communication/tcp/TcpCommunicationSpiMBean.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/spi/communication/tcp/TcpCommunicationSpiMBean.java b/modules/core/src/main/java/org/apache/ignite/spi/communication/tcp/TcpCommunicationSpiMBean.java
index 5c80e6e..6f5a738 100644
--- a/modules/core/src/main/java/org/apache/ignite/spi/communication/tcp/TcpCommunicationSpiMBean.java
+++ b/modules/core/src/main/java/org/apache/ignite/spi/communication/tcp/TcpCommunicationSpiMBean.java
@@ -171,8 +171,6 @@ public interface TcpCommunicationSpiMBean extends IgniteSpiManagementMBean {
      * This frequency defines how often system will advice to flush
      * connection buffer.
      * <p>
-     * If not provided, default value is {@link TcpCommunicationSpi#DFLT_CONN_BUF_FLUSH_FREQ}.
-     * <p>
      * This property is used only if {@link #getConnectionBufferSize()} is greater than {@code 0}.
      *
      * @param connBufFlushFreq Flush frequency.

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/fb827a77/modules/core/src/test/java/org/apache/ignite/internal/util/nio/GridNioSelfTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/internal/util/nio/GridNioSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/util/nio/GridNioSelfTest.java
index e3baeb0..bdf9929 100644
--- a/modules/core/src/test/java/org/apache/ignite/internal/util/nio/GridNioSelfTest.java
+++ b/modules/core/src/test/java/org/apache/ignite/internal/util/nio/GridNioSelfTest.java
@@ -1286,7 +1286,7 @@ public class GridNioSelfTest extends GridCommonAbstractTest {
     }
 
     /**
-     * Test client to use instead of {@link GridTcpCommunicationClient}
+     * Test client to use instead of {@link GridTcpNioCommunicationClient}
      */
     private static class TestClient implements AutoCloseable {
         /** Socket implementation to use. */

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/fb827a77/modules/core/src/test/java/org/apache/ignite/spi/communication/tcp/GridTcpCommunicationSpiAbstractTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/spi/communication/tcp/GridTcpCommunicationSpiAbstractTest.java b/modules/core/src/test/java/org/apache/ignite/spi/communication/tcp/GridTcpCommunicationSpiAbstractTest.java
index ea51aff..8d27485 100644
--- a/modules/core/src/test/java/org/apache/ignite/spi/communication/tcp/GridTcpCommunicationSpiAbstractTest.java
+++ b/modules/core/src/test/java/org/apache/ignite/spi/communication/tcp/GridTcpCommunicationSpiAbstractTest.java
@@ -64,7 +64,7 @@ abstract class GridTcpCommunicationSpiAbstractTest extends GridAbstractCommunica
 
         // Test idle clients remove.
         for (CommunicationSpi spi : spis.values()) {
-            ConcurrentMap<UUID, GridTcpCommunicationClient> clients = U.field(spi, "clients");
+            ConcurrentMap<UUID, GridCommunicationClient> clients = U.field(spi, "clients");
 
             assertEquals(2, clients.size());
 
@@ -77,7 +77,7 @@ abstract class GridTcpCommunicationSpiAbstractTest extends GridAbstractCommunica
         super.afterTest();
 
         for (CommunicationSpi spi : spis.values()) {
-            ConcurrentMap<UUID, GridTcpCommunicationClient> clients = U.field(spi, "clients");
+            ConcurrentMap<UUID, GridCommunicationClient> clients = U.field(spi, "clients");
 
             for (int i = 0; i < 20 && !clients.isEmpty(); i++) {
                 info("Check failed for SPI [grid=" +

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/fb827a77/modules/core/src/test/java/org/apache/ignite/spi/communication/tcp/GridTcpCommunicationSpiConcurrentConnectSelfTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/spi/communication/tcp/GridTcpCommunicationSpiConcurrentConnectSelfTest.java b/modules/core/src/test/java/org/apache/ignite/spi/communication/tcp/GridTcpCommunicationSpiConcurrentConnectSelfTest.java
index c038ee7..2d175f5 100644
--- a/modules/core/src/test/java/org/apache/ignite/spi/communication/tcp/GridTcpCommunicationSpiConcurrentConnectSelfTest.java
+++ b/modules/core/src/test/java/org/apache/ignite/spi/communication/tcp/GridTcpCommunicationSpiConcurrentConnectSelfTest.java
@@ -256,7 +256,7 @@ public class GridTcpCommunicationSpiConcurrentConnectSelfTest<T extends Communic
                     assertTrue(latch.await(10, TimeUnit.SECONDS));
 
                     for (CommunicationSpi spi : spis) {
-                        ConcurrentMap<UUID, GridTcpCommunicationClient> clients = U.field(spi, "clients");
+                        ConcurrentMap<UUID, GridCommunicationClient> clients = U.field(spi, "clients");
 
                         assertEquals(1, clients.size());
 

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/fb827a77/modules/core/src/test/java/org/apache/ignite/spi/communication/tcp/GridTcpCommunicationSpiConfigSelfTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/spi/communication/tcp/GridTcpCommunicationSpiConfigSelfTest.java b/modules/core/src/test/java/org/apache/ignite/spi/communication/tcp/GridTcpCommunicationSpiConfigSelfTest.java
index 34fa610..c4a0916 100644
--- a/modules/core/src/test/java/org/apache/ignite/spi/communication/tcp/GridTcpCommunicationSpiConfigSelfTest.java
+++ b/modules/core/src/test/java/org/apache/ignite/spi/communication/tcp/GridTcpCommunicationSpiConfigSelfTest.java
@@ -32,8 +32,6 @@ public class GridTcpCommunicationSpiConfigSelfTest extends GridSpiAbstractConfig
         checkNegativeSpiProperty(new TcpCommunicationSpi(), "localPort", 65636);
         checkNegativeSpiProperty(new TcpCommunicationSpi(), "localPortRange", -1);
         checkNegativeSpiProperty(new TcpCommunicationSpi(), "idleConnectionTimeout", 0);
-        checkNegativeSpiProperty(new TcpCommunicationSpi(), "connectionBufferSize", -1);
-        checkNegativeSpiProperty(new TcpCommunicationSpi(), "connectionBufferFlushFrequency", 0);
         checkNegativeSpiProperty(new TcpCommunicationSpi(), "socketReceiveBuffer", -1);
         checkNegativeSpiProperty(new TcpCommunicationSpi(), "socketSendBuffer", -1);
         checkNegativeSpiProperty(new TcpCommunicationSpi(), "messageQueueLimit", -1);

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/fb827a77/modules/core/src/test/java/org/apache/ignite/spi/communication/tcp/GridTcpCommunicationSpiMultithreadedSelfTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/spi/communication/tcp/GridTcpCommunicationSpiMultithreadedSelfTest.java b/modules/core/src/test/java/org/apache/ignite/spi/communication/tcp/GridTcpCommunicationSpiMultithreadedSelfTest.java
index e7ae957..3916f02 100644
--- a/modules/core/src/test/java/org/apache/ignite/spi/communication/tcp/GridTcpCommunicationSpiMultithreadedSelfTest.java
+++ b/modules/core/src/test/java/org/apache/ignite/spi/communication/tcp/GridTcpCommunicationSpiMultithreadedSelfTest.java
@@ -501,7 +501,7 @@ public class GridTcpCommunicationSpiMultithreadedSelfTest extends GridSpiAbstrac
         }
 
         for (CommunicationSpi spi : spis.values()) {
-            final ConcurrentMap<UUID, GridTcpCommunicationClient> clients = U.field(spi, "clients");
+            final ConcurrentMap<UUID, GridCommunicationClient> clients = U.field(spi, "clients");
 
             assert GridTestUtils.waitForCondition(new PA() {
                 @Override public boolean apply() {

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/fb827a77/modules/core/src/test/java/org/apache/ignite/spi/discovery/AbstractDiscoverySelfTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/spi/discovery/AbstractDiscoverySelfTest.java b/modules/core/src/test/java/org/apache/ignite/spi/discovery/AbstractDiscoverySelfTest.java
index 9c6fbb4..61bb944 100644
--- a/modules/core/src/test/java/org/apache/ignite/spi/discovery/AbstractDiscoverySelfTest.java
+++ b/modules/core/src/test/java/org/apache/ignite/spi/discovery/AbstractDiscoverySelfTest.java
@@ -22,6 +22,7 @@ import org.apache.ignite.cluster.*;
 import org.apache.ignite.internal.util.typedef.internal.*;
 import org.apache.ignite.marshaller.*;
 import org.apache.ignite.spi.*;
+import org.apache.ignite.testframework.*;
 import org.apache.ignite.testframework.config.*;
 import org.apache.ignite.testframework.junits.*;
 import org.apache.ignite.testframework.junits.spi.*;
@@ -267,9 +268,8 @@ public abstract class AbstractDiscoverySelfTest<T extends IgniteSpi> extends Gri
 
             Collection<UUID> nodeIds = new HashSet<>();
 
-            for (IgniteTestResources rsrc : spiRsrcs) {
+            for (IgniteTestResources rsrc : spiRsrcs)
                 nodeIds.add(rsrc.getNodeId());
-            }
 
             for (ClusterNode node : spi.getRemoteNodes()) {
                 if (nodeIds.contains(node.id())) {
@@ -390,6 +390,10 @@ public abstract class AbstractDiscoverySelfTest<T extends IgniteSpi> extends Gri
                     }
                 });
 
+                GridSpiTestContext ctx = initSpiContext();
+
+                GridTestUtils.setFieldValue(spi, IgniteSpiAdapter.class, "spiCtx", ctx);
+
                 spi.spiStart(getTestGridName() + i);
 
                 spis.add(spi);
@@ -397,7 +401,7 @@ public abstract class AbstractDiscoverySelfTest<T extends IgniteSpi> extends Gri
                 spiRsrcs.add(rsrcMgr);
 
                 // Force to use test context instead of default dummy context.
-                spi.onContextInitialized(initSpiContext());
+                spi.onContextInitialized(ctx);
             }
         }
         catch (Throwable e) {
@@ -438,9 +442,8 @@ public abstract class AbstractDiscoverySelfTest<T extends IgniteSpi> extends Gri
             spi.spiStop();
         }
 
-        for (IgniteTestResources rscrs : spiRsrcs) {
+        for (IgniteTestResources rscrs : spiRsrcs)
             rscrs.stopThreads();
-        }
 
         // Clear.
         spis.clear();



[31/42] incubator-ignite git commit: # ignite-883 minor

Posted by ag...@apache.org.
# ignite-883 minor


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

Branch: refs/heads/ignite-389
Commit: 730ef104ad8938c88793c374abd2396cf8899f17
Parents: 0a4e7dd
Author: sboikov <sb...@gridgain.com>
Authored: Mon Jun 8 10:19:46 2015 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Mon Jun 8 10:19:46 2015 +0300

----------------------------------------------------------------------
 .../ignite/internal/util/nio/GridNioRecoveryDescriptor.java      | 4 ++--
 1 file changed, 2 insertions(+), 2 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/730ef104/modules/core/src/main/java/org/apache/ignite/internal/util/nio/GridNioRecoveryDescriptor.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/util/nio/GridNioRecoveryDescriptor.java b/modules/core/src/main/java/org/apache/ignite/internal/util/nio/GridNioRecoveryDescriptor.java
index 3bc81ea..733ae81 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/util/nio/GridNioRecoveryDescriptor.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/util/nio/GridNioRecoveryDescriptor.java
@@ -244,9 +244,9 @@ public class GridNioRecoveryDescriptor {
         synchronized (this) {
             if (!nodeLeft)
                 ackReceived(rcvCnt);
-        }
 
-        resendCnt = msgFuts.size();
+            resendCnt = msgFuts.size();
+        }
     }
 
     /**


[24/42] incubator-ignite git commit: # ignite-sprint-5 disabled hanging test

Posted by ag...@apache.org.
# ignite-sprint-5 disabled hanging test


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

Branch: refs/heads/ignite-389
Commit: 5f06f578e57001e964d823daebc3bc67b1a27675
Parents: e87969c
Author: sboikov <sb...@gridgain.com>
Authored: Fri Jun 5 17:21:23 2015 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Fri Jun 5 17:21:23 2015 +0300

----------------------------------------------------------------------
 .../org/apache/ignite/messaging/IgniteMessagingWithClientTest.java | 2 ++
 1 file changed, 2 insertions(+)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/5f06f578/modules/core/src/test/java/org/apache/ignite/messaging/IgniteMessagingWithClientTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/messaging/IgniteMessagingWithClientTest.java b/modules/core/src/test/java/org/apache/ignite/messaging/IgniteMessagingWithClientTest.java
index 855a4f7..09abcdb 100644
--- a/modules/core/src/test/java/org/apache/ignite/messaging/IgniteMessagingWithClientTest.java
+++ b/modules/core/src/test/java/org/apache/ignite/messaging/IgniteMessagingWithClientTest.java
@@ -77,6 +77,8 @@ public class IgniteMessagingWithClientTest extends GridCommonAbstractTest implem
      * @throws Exception If failed.
      */
     public void testMessageSendWithClientJoin() throws Exception {
+        fail("https://issues.apache.org/jira/browse/IGNITE-996");
+
         startGrid(0);
 
         Ignite ignite1 = startGrid(1);


[08/42] incubator-ignite git commit: IGNITE-941 - Read-only mode for transactional cache

Posted by ag...@apache.org.
IGNITE-941 - Read-only mode for transactional cache


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

Branch: refs/heads/ignite-389
Commit: 54c72c3e7fb89a3f9e6457dd82aa83af1af4ba1c
Parents: 97d0bc1
Author: Valentin Kulichenko <vk...@gridgain.com>
Authored: Wed Jun 3 20:06:33 2015 -0700
Committer: Valentin Kulichenko <vk...@gridgain.com>
Committed: Wed Jun 3 20:06:33 2015 -0700

----------------------------------------------------------------------
 .../cache/DynamicCacheDescriptor.java           |  17 ++
 .../processors/cache/GridCacheContext.java      |  16 +-
 .../processors/cache/GridCacheProcessor.java    |  37 ++-
 .../dht/atomic/GridDhtAtomicCache.java          |   4 +
 .../transactions/IgniteTxLocalAdapter.java      |   7 +
 ...acheReadOnlyTransactionalClientSelfTest.java | 289 +++++++++++++++++++
 .../loadtests/hashmap/GridCacheTestContext.java |   1 +
 .../testsuites/IgniteCacheTestSuite4.java       |   2 +
 8 files changed, 362 insertions(+), 11 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/54c72c3e/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/DynamicCacheDescriptor.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/DynamicCacheDescriptor.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/DynamicCacheDescriptor.java
index a27ebd4..9c6cc43 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/DynamicCacheDescriptor.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/DynamicCacheDescriptor.java
@@ -61,6 +61,9 @@ public class DynamicCacheDescriptor {
     /** Cache plugin manager. */
     private final CachePluginManager pluginMgr;
 
+    /** */
+    private boolean updatesAllowed = true;
+
     /**
      * @param ctx Context.
      * @param cacheCfg Cache configuration.
@@ -208,6 +211,20 @@ public class DynamicCacheDescriptor {
         rmtCfgs = null;
     }
 
+    /**
+     * @return Updates allowed flag.
+     */
+    public boolean updatesAllowed() {
+        return updatesAllowed;
+    }
+
+    /**
+     * @param updatesAllowed Updates allowed flag.
+     */
+    public void updatesAllowed(boolean updatesAllowed) {
+        this.updatesAllowed = updatesAllowed;
+    }
+
     /** {@inheritDoc} */
     @Override public String toString() {
         return S.toString(DynamicCacheDescriptor.class, this, "cacheName", U.maskName(cacheCfg.getName()));

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/54c72c3e/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheContext.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheContext.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheContext.java
index b20e59d..8a4e3b9 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheContext.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheContext.java
@@ -196,6 +196,9 @@ public class GridCacheContext<K, V> implements Externalizable {
     /** Dynamic cache deployment ID. */
     private IgniteUuid dynamicDeploymentId;
 
+    /** Updates allowed flag. */
+    private boolean updatesAllowed;
+
     /**
      * Empty constructor required for {@link Externalizable}.
      */
@@ -209,6 +212,7 @@ public class GridCacheContext<K, V> implements Externalizable {
      * @param cacheCfg Cache configuration.
      * @param cacheType Cache type.
      * @param affNode {@code True} if local node is affinity node.
+     * @param updatesAllowed Updates allowed flag.
      * @param evtMgr Cache event manager.
      * @param swapMgr Cache swap manager.
      * @param storeMgr Store manager.
@@ -230,6 +234,7 @@ public class GridCacheContext<K, V> implements Externalizable {
         CacheConfiguration cacheCfg,
         CacheType cacheType,
         boolean affNode,
+        boolean updatesAllowed,
 
         /*
          * Managers in starting order!
@@ -271,6 +276,7 @@ public class GridCacheContext<K, V> implements Externalizable {
         this.cacheCfg = cacheCfg;
         this.cacheType = cacheType;
         this.affNode = affNode;
+        this.updatesAllowed = updatesAllowed;
 
         /*
          * Managers in starting order!
@@ -1469,9 +1475,6 @@ public class GridCacheContext<K, V> implements Externalizable {
             Collection<ClusterNode> dhtNodeIds = new ArrayList<>(dhtRemoteNodes);
             Collection<ClusterNode> nearNodeIds = F.isEmpty(nearRemoteNodes) ? null : new ArrayList<>(nearRemoteNodes);
 
-            if (!F.isEmpty(nearNodeIds))
-                U.dumpStack("Added near mapped nodes: " + entry + ", " + nearNodeIds);
-
             entry.mappings(explicitLockVer, dhtNodeIds, nearNodeIds);
         }
 
@@ -1809,6 +1812,13 @@ public class GridCacheContext<K, V> implements Externalizable {
     }
 
     /**
+     * @return Updates allowed.
+     */
+    public boolean updatesAllowed() {
+        return updatesAllowed;
+    }
+
+    /**
      * Nulling references to potentially leak-prone objects.
      */
     public void cleanup() {

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/54c72c3e/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheProcessor.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheProcessor.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheProcessor.java
index 24e2439..bbd5fe9 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheProcessor.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheProcessor.java
@@ -674,6 +674,9 @@ public class GridCacheProcessor extends GridProcessorAdapter {
 
             if (!getBoolean(IGNITE_SKIP_CONFIGURATION_CONSISTENCY_CHECK)) {
                 for (ClusterNode n : ctx.discovery().remoteNodes()) {
+                    if (Boolean.valueOf(n.<String>attribute(IGNITE_SKIP_CONFIGURATION_CONSISTENCY_CHECK)))
+                        continue;
+
                     checkTransactionConfiguration(n);
 
                     DeploymentMode locDepMode = ctx.config().getDeploymentMode();
@@ -688,7 +691,7 @@ public class GridCacheProcessor extends GridProcessorAdapter {
                         if (rmtCfg != null) {
                             CacheConfiguration locCfg = desc.cacheConfiguration();
 
-                            checkCache(locCfg, rmtCfg, n);
+                            checkCache(locCfg, rmtCfg, n, desc);
 
                             // Check plugin cache configurations.
                             CachePluginManager pluginMgr = desc.pluginManager();
@@ -718,7 +721,8 @@ public class GridCacheProcessor extends GridProcessorAdapter {
 
                     CachePluginManager pluginMgr = desc.pluginManager();
 
-                    GridCacheContext ctx = createCache(ccfg, pluginMgr, desc.cacheType(), cacheObjCtx);
+                    GridCacheContext ctx = createCache(
+                        ccfg, pluginMgr, desc.cacheType(), cacheObjCtx, desc.updatesAllowed());
 
                     ctx.dynamicDeploymentId(desc.deploymentId());
 
@@ -1068,7 +1072,8 @@ public class GridCacheProcessor extends GridProcessorAdapter {
     private GridCacheContext createCache(CacheConfiguration<?, ?> cfg,
         @Nullable CachePluginManager pluginMgr,
         CacheType cacheType,
-        CacheObjectContext cacheObjCtx)
+        CacheObjectContext cacheObjCtx,
+        boolean updatesAllowed)
         throws IgniteCheckedException
     {
         assert cfg != null;
@@ -1126,6 +1131,7 @@ public class GridCacheProcessor extends GridProcessorAdapter {
             cfg,
             cacheType,
             ctx.discovery().cacheAffinityNode(ctx.discovery().localNode(), cfg.getName()),
+            updatesAllowed,
 
             /*
              * Managers in starting order!
@@ -1255,6 +1261,7 @@ public class GridCacheProcessor extends GridProcessorAdapter {
                 cfg,
                 cacheType,
                 ctx.discovery().cacheAffinityNode(ctx.discovery().localNode(), cfg.getName()),
+                true,
 
                 /*
                  * Managers in starting order!
@@ -1458,7 +1465,7 @@ public class GridCacheProcessor extends GridProcessorAdapter {
 
             CacheObjectContext cacheObjCtx = ctx.cacheObjects().contextForCache(ccfg);
 
-            GridCacheContext cacheCtx = createCache(ccfg, null, cacheType, cacheObjCtx);
+            GridCacheContext cacheCtx = createCache(ccfg, null, cacheType, cacheObjCtx, true);
 
             cacheCtx.startTopologyVersion(topVer);
 
@@ -2229,10 +2236,11 @@ public class GridCacheProcessor extends GridProcessorAdapter {
      * @param locCfg Local configuration.
      * @param rmtCfg Remote configuration.
      * @param rmtNode Remote node.
+     * @param desc Cache descriptor.
      * @throws IgniteCheckedException If check failed.
      */
-    private void checkCache(CacheConfiguration locCfg, CacheConfiguration rmtCfg, ClusterNode rmtNode)
-        throws IgniteCheckedException {
+    private void checkCache(CacheConfiguration locCfg, CacheConfiguration rmtCfg, ClusterNode rmtNode,
+        DynamicCacheDescriptor desc) throws IgniteCheckedException {
         ClusterNode locNode = ctx.discovery().localNode();
 
         UUID rmt = rmtNode.id();
@@ -2240,6 +2248,9 @@ public class GridCacheProcessor extends GridProcessorAdapter {
         GridCacheAttributes rmtAttr = new GridCacheAttributes(rmtCfg);
         GridCacheAttributes locAttr = new GridCacheAttributes(locCfg);
 
+        boolean isLocAff = CU.affinityNode(locNode, locCfg.getNodeFilter());
+        boolean isRmtAff = CU.affinityNode(rmtNode, rmtCfg.getNodeFilter());
+
         CU.checkAttributeMismatch(log, rmtAttr.cacheName(), rmt, "cacheMode", "Cache mode",
             locAttr.cacheMode(), rmtAttr.cacheMode(), true);
 
@@ -2253,8 +2264,18 @@ public class GridCacheProcessor extends GridProcessorAdapter {
             CU.checkAttributeMismatch(log, rmtAttr.cacheName(), rmt, "cachePreloadMode",
                 "Cache preload mode", locAttr.cacheRebalanceMode(), rmtAttr.cacheRebalanceMode(), true);
 
-            if (locCfg.getAtomicityMode() == TRANSACTIONAL ||
-                (CU.affinityNode(rmtNode, rmtCfg.getNodeFilter()) && CU.affinityNode(locNode, locCfg.getNodeFilter())))
+            boolean checkStore;
+
+            if (!isLocAff && isRmtAff && locCfg.getAtomicityMode() == TRANSACTIONAL) {
+                checkStore = locAttr.storeFactoryClassName() != null;
+
+                if (locAttr.storeFactoryClassName() == null && rmtAttr.storeFactoryClassName() != null)
+                    desc.updatesAllowed(false);
+            }
+            else
+                checkStore = isLocAff && isRmtAff;
+
+            if (checkStore)
                 CU.checkAttributeMismatch(log, rmtAttr.cacheName(), rmt, "storeFactory", "Store factory",
                     locAttr.storeFactoryClassName(), rmtAttr.storeFactoryClassName(), true);
 

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/54c72c3e/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 9ca80f9..3208aeb 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
@@ -737,6 +737,8 @@ public class GridDhtAtomicCache<K, V> extends GridDhtCacheAdapter<K, V> {
         @Nullable final CacheEntryPredicate[] filter,
         final boolean waitTopFut
     ) {
+        assert ctx.updatesAllowed();
+
         if (map != null && keyCheck)
             validateCacheKeys(map.keySet());
 
@@ -793,6 +795,8 @@ public class GridDhtAtomicCache<K, V> extends GridDhtCacheAdapter<K, V> {
         boolean rawRetval,
         @Nullable final CacheEntryPredicate[] filter
     ) {
+        assert ctx.updatesAllowed();
+
         final boolean statsEnabled = ctx.config().isStatisticsEnabled();
 
         final long start = statsEnabled ? System.nanoTime() : 0L;

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/54c72c3e/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/transactions/IgniteTxLocalAdapter.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/transactions/IgniteTxLocalAdapter.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/transactions/IgniteTxLocalAdapter.java
index 3e5a63c..4063be2 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/transactions/IgniteTxLocalAdapter.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/transactions/IgniteTxLocalAdapter.java
@@ -40,6 +40,7 @@ import org.apache.ignite.plugin.security.*;
 import org.apache.ignite.transactions.*;
 import org.jetbrains.annotations.*;
 
+import javax.cache.*;
 import javax.cache.expiry.*;
 import javax.cache.processor.*;
 import java.io.*;
@@ -2560,6 +2561,9 @@ public abstract class IgniteTxLocalAdapter extends IgniteTxAdapter
     ) {
         assert filter == null || invokeMap == null;
 
+        if (!cacheCtx.updatesAllowed())
+            throw new CacheException("Updates are not allowed for cache: " + cacheCtx.name());
+
         cacheCtx.checkSecurity(SecurityPermission.CACHE_PUT);
 
         if (retval)
@@ -2780,6 +2784,9 @@ public abstract class IgniteTxLocalAdapter extends IgniteTxAdapter
         @Nullable GridCacheEntryEx cached,
         final boolean retval,
         @Nullable final CacheEntryPredicate[] filter) {
+        if (!cacheCtx.updatesAllowed())
+            throw new CacheException("Updates are not allowed for cache: " + cacheCtx.name());
+
         cacheCtx.checkSecurity(SecurityPermission.CACHE_REMOVE);
 
         if (retval)

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/54c72c3e/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheReadOnlyTransactionalClientSelfTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheReadOnlyTransactionalClientSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheReadOnlyTransactionalClientSelfTest.java
new file mode 100644
index 0000000..e2a1def
--- /dev/null
+++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheReadOnlyTransactionalClientSelfTest.java
@@ -0,0 +1,289 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License.  You may obtain a copy of the License at
+ *
+ *      http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.ignite.internal.processors.cache;
+
+import org.apache.ignite.*;
+import org.apache.ignite.cache.*;
+import org.apache.ignite.cache.store.*;
+import org.apache.ignite.configuration.*;
+import org.apache.ignite.internal.util.typedef.*;
+import org.apache.ignite.spi.discovery.tcp.*;
+import org.apache.ignite.spi.discovery.tcp.ipfinder.*;
+import org.apache.ignite.spi.discovery.tcp.ipfinder.vm.*;
+import org.apache.ignite.testframework.junits.common.*;
+
+import javax.cache.*;
+import javax.cache.configuration.*;
+import javax.cache.processor.*;
+
+/**
+ * Tests for read-only transactional cache client.
+ */
+public class CacheReadOnlyTransactionalClientSelfTest extends GridCommonAbstractTest {
+    /** */
+    private static final TcpDiscoveryIpFinder IP_FINDER = new TcpDiscoveryVmIpFinder(true);
+
+    /** */
+    private boolean client;
+
+    /** */
+    private boolean nearEnabled;
+
+    /** */
+    private Factory<CacheStore> factory;
+
+    /** {@inheritDoc} */
+    @Override protected IgniteConfiguration getConfiguration(String gridName) throws Exception {
+        IgniteConfiguration cfg = super.getConfiguration(gridName);
+
+        cfg.setClientMode(client);
+
+        CacheConfiguration cc = new CacheConfiguration();
+
+        cc.setAtomicityMode(CacheAtomicityMode.TRANSACTIONAL);
+        cc.setCacheStoreFactory(factory);
+
+        if (client && nearEnabled)
+            cc.setNearConfiguration(new NearCacheConfiguration());
+
+        cfg.setCacheConfiguration(cc);
+
+        TcpDiscoverySpi disco = new TcpDiscoverySpi();
+
+        disco.setIpFinder(IP_FINDER);
+
+        cfg.setDiscoverySpi(disco);
+
+        return cfg;
+    }
+
+    /** {@inheritDoc} */
+    @Override protected void beforeTestsStarted() throws Exception {
+        client = false;
+        factory = new Factory1();
+
+        startGrids(2);
+    }
+
+    /** {@inheritDoc} */
+    @Override protected void afterTestsStopped() throws Exception {
+        stopAllGrids();
+    }
+
+    /** {@inheritDoc} */
+    @Override protected void afterTest() throws Exception {
+        stopGrid();
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testCorrectStore() throws Exception {
+        client = true;
+        nearEnabled = false;
+        factory = new Factory1();
+
+        Ignite ignite = startGrid();
+
+        IgniteCache cache = ignite.cache(null);
+
+        cache.get(0);
+        cache.getAll(F.asSet(0, 1));
+        cache.getAndPut(0, 0);
+        cache.getAndPutIfAbsent(0, 0);
+        cache.getAndRemove(0);
+        cache.getAndReplace(0, 0);
+        cache.put(0, 0);
+        cache.putAll(F.asMap(0, 0, 1, 1));
+        cache.putIfAbsent(0, 0);
+        cache.remove(0);
+        cache.remove(0, 0);
+        cache.removeAll(F.asSet(0, 1));
+        cache.removeAll();
+        cache.invoke(0, new EP());
+        cache.invokeAll(F.asSet(0, 1), new EP());
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testInvalidStore() throws Exception {
+        client = true;
+        nearEnabled = false;
+        factory = new Factory2();
+
+        try {
+            startGrid();
+
+            assert false : "Exception was not thrown.";
+        }
+        catch (Exception e) {
+            assert e.getMessage().startsWith("Store factory mismatch") : e.getMessage();
+        }
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testNoStoreNearDisabled() throws Exception {
+        nearEnabled = false;
+
+        doTestNoStore();
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testNoStoreNearEnabled() throws Exception {
+        nearEnabled = true;
+
+        doTestNoStore();
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    private void doTestNoStore() throws Exception {
+        client = true;
+        factory = null;
+
+        Ignite ignite = startGrid();
+
+        IgniteCache cache = ignite.cache(null);
+
+        cache.get(0);
+        cache.getAll(F.asSet(0, 1));
+
+        try {
+            cache.getAndPut(0, 0);
+        }
+        catch (CacheException e) {
+            assertEquals("Updates are not allowed for cache: null", e.getMessage());
+        }
+
+        try {
+            cache.getAndPutIfAbsent(0, 0);
+        }
+        catch (CacheException e) {
+            assertEquals("Updates are not allowed for cache: null", e.getMessage());
+        }
+
+        try {
+            cache.getAndRemove(0);
+        }
+        catch (CacheException e) {
+            assertEquals("Updates are not allowed for cache: null", e.getMessage());
+        }
+
+        try {
+            cache.getAndReplace(0, 0);
+        }
+        catch (CacheException e) {
+            assertEquals("Updates are not allowed for cache: null", e.getMessage());
+        }
+
+        try {
+            cache.put(0, 0);
+        }
+        catch (CacheException e) {
+            assertEquals("Updates are not allowed for cache: null", e.getMessage());
+        }
+
+        try {
+            cache.putAll(F.asMap(0, 0, 1, 1));
+        }
+        catch (CacheException e) {
+            assertEquals("Updates are not allowed for cache: null", e.getMessage());
+        }
+
+        try {
+            cache.putIfAbsent(0, 0);
+        }
+        catch (CacheException e) {
+            assertEquals("Updates are not allowed for cache: null", e.getMessage());
+        }
+
+        try {
+            cache.remove(0);
+        }
+        catch (CacheException e) {
+            assertEquals("Updates are not allowed for cache: null", e.getMessage());
+        }
+
+        try {
+            cache.remove(0, 0);
+        }
+        catch (CacheException e) {
+            assertEquals("Updates are not allowed for cache: null", e.getMessage());
+        }
+
+        try {
+            cache.removeAll(F.asSet(0, 1));
+        }
+        catch (CacheException e) {
+            assertEquals("Updates are not allowed for cache: null", e.getMessage());
+        }
+
+        try {
+            cache.removeAll();
+        }
+        catch (CacheException e) {
+            assertEquals("Updates are not allowed for cache: null", e.getMessage());
+        }
+
+        try {
+            cache.invoke(0, new EP());
+        }
+        catch (CacheException e) {
+            assertEquals("Updates are not allowed for cache: null", e.getMessage());
+        }
+
+        try {
+            cache.invokeAll(F.asSet(0, 1), new EP());
+        }
+        catch (CacheException e) {
+            assertEquals("Updates are not allowed for cache: null", e.getMessage());
+        }
+    }
+
+    /**
+     */
+    private static class Factory1 implements Factory<CacheStore> {
+        /** {@inheritDoc} */
+        @Override public CacheStore create() {
+            return null;
+        }
+    }
+
+    /**
+     */
+    private static class Factory2 implements Factory<CacheStore> {
+        /** {@inheritDoc} */
+        @Override public CacheStore create() {
+            return null;
+        }
+    }
+
+    /**
+     */
+    private static class EP implements CacheEntryProcessor {
+        @Override public Object process(MutableEntry entry, Object... arguments) {
+            return null;
+        }
+    }
+}

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/54c72c3e/modules/core/src/test/java/org/apache/ignite/loadtests/hashmap/GridCacheTestContext.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/loadtests/hashmap/GridCacheTestContext.java b/modules/core/src/test/java/org/apache/ignite/loadtests/hashmap/GridCacheTestContext.java
index 1c85ed3..77d3905 100644
--- a/modules/core/src/test/java/org/apache/ignite/loadtests/hashmap/GridCacheTestContext.java
+++ b/modules/core/src/test/java/org/apache/ignite/loadtests/hashmap/GridCacheTestContext.java
@@ -60,6 +60,7 @@ public class GridCacheTestContext<K, V> extends GridCacheContext<K, V> {
             defaultCacheConfiguration(),
             CacheType.USER,
             true,
+            true,
             new GridCacheEventManager(),
             new GridCacheSwapManager(false),
             new CacheOsStoreManager(null, new CacheConfiguration()),

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/54c72c3e/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteCacheTestSuite4.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteCacheTestSuite4.java b/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteCacheTestSuite4.java
index a8019d2..ab304fb 100644
--- a/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteCacheTestSuite4.java
+++ b/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteCacheTestSuite4.java
@@ -135,6 +135,8 @@ public class IgniteCacheTestSuite4 extends TestSuite {
 
         suite.addTestSuite(CacheJdbcStoreSessionListenerSelfTest.class);
 
+        suite.addTestSuite(CacheReadOnlyTransactionalClientSelfTest.class);
+
         return suite;
     }
 }


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

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


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

Branch: refs/heads/ignite-389
Commit: e9cea3a311cd1ad9728a0ae14de9ba91f3859d1b
Parents: 54c72c3 bd3abbc
Author: Valentin Kulichenko <vk...@gridgain.com>
Authored: Wed Jun 3 20:06:43 2015 -0700
Committer: Valentin Kulichenko <vk...@gridgain.com>
Committed: Wed Jun 3 20:06:43 2015 -0700

----------------------------------------------------------------------

----------------------------------------------------------------------



[41/42] incubator-ignite git commit: # ignite-sprint-5 minor optimization in force keys future

Posted by ag...@apache.org.
# ignite-sprint-5 minor optimization in force keys future


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

Branch: refs/heads/ignite-389
Commit: 0fa2853e060fee7c9c8c8484be412e91d30c52da
Parents: ff7827e
Author: sboikov <sb...@gridgain.com>
Authored: Mon Jun 8 16:31:31 2015 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Mon Jun 8 16:31:31 2015 +0300

----------------------------------------------------------------------
 .../dht/preloader/GridDhtForceKeysFuture.java   | 40 +++++++++++++-------
 1 file changed, 26 insertions(+), 14 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/0fa2853e/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/preloader/GridDhtForceKeysFuture.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/preloader/GridDhtForceKeysFuture.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/preloader/GridDhtForceKeysFuture.java
index 9637fd1..1d57ef7 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/preloader/GridDhtForceKeysFuture.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/preloader/GridDhtForceKeysFuture.java
@@ -208,21 +208,21 @@ public final class GridDhtForceKeysFuture<K, V> extends GridCompoundFuture<Objec
      * @return {@code True} if some mapping was added.
      */
     private boolean map(Iterable<KeyCacheObject> keys, Collection<ClusterNode> exc) {
-        Map<ClusterNode, Set<KeyCacheObject>> mappings = new HashMap<>();
-
-        ClusterNode loc = cctx.localNode();
-
-        int curTopVer = topCntr.get();
+        Map<ClusterNode, Set<KeyCacheObject>> mappings = null;
 
         for (KeyCacheObject key : keys)
-            map(key, mappings, exc);
+            mappings = map(key, mappings, exc);
 
         if (isDone())
             return false;
 
         boolean ret = false;
 
-        if (!mappings.isEmpty()) {
+        if (mappings != null) {
+            ClusterNode loc = cctx.localNode();
+
+            int curTopVer = topCntr.get();
+
             preloader.addFuture(this);
 
             trackable = true;
@@ -275,22 +275,27 @@ public final class GridDhtForceKeysFuture<K, V> extends GridCompoundFuture<Objec
      * @param key Key.
      * @param exc Exclude nodes.
      * @param mappings Mappings.
+     * @return Mappings.
      */
-    private void map(KeyCacheObject key, Map<ClusterNode, Set<KeyCacheObject>> mappings, Collection<ClusterNode> exc) {
+    private Map<ClusterNode, Set<KeyCacheObject>> map(KeyCacheObject key,
+        @Nullable Map<ClusterNode, Set<KeyCacheObject>> mappings,
+        Collection<ClusterNode> exc)
+    {
         ClusterNode loc = cctx.localNode();
 
-        int part = cctx.affinity().partition(key);
-
         GridCacheEntryEx e = cctx.dht().peekEx(key);
 
         try {
             if (e != null && !e.isNewLocked()) {
-                if (log.isDebugEnabled())
+                if (log.isDebugEnabled()) {
+                    int part = cctx.affinity().partition(key);
+
                     log.debug("Will not rebalance key (entry is not new) [cacheName=" + cctx.name() +
                         ", key=" + key + ", part=" + part + ", locId=" + cctx.nodeId() + ']');
+                }
 
                 // Key has been rebalanced or retrieved already.
-                return;
+                return mappings;
             }
         }
         catch (GridCacheEntryRemovedException ignore) {
@@ -299,6 +304,8 @@ public final class GridDhtForceKeysFuture<K, V> extends GridCompoundFuture<Objec
                     ", locId=" + cctx.nodeId() + ']');
         }
 
+        int part = cctx.affinity().partition(key);
+
         List<ClusterNode> owners = F.isEmpty(exc) ? top.owners(part, topVer) :
             new ArrayList<>(F.view(top.owners(part, topVer), F.notIn(exc)));
 
@@ -308,7 +315,7 @@ public final class GridDhtForceKeysFuture<K, V> extends GridCompoundFuture<Objec
                     "topVer=" + topVer + ", locId=" + cctx.nodeId() + ']');
 
             // Key is already rebalanced.
-            return;
+            return mappings;
         }
 
         // Create partition.
@@ -337,9 +344,12 @@ public final class GridDhtForceKeysFuture<K, V> extends GridCompoundFuture<Objec
                     log.debug("Will not rebalance key (no nodes to request from with rebalancing disabled) [key=" +
                         key + ", part=" + part + ", locId=" + cctx.nodeId() + ']');
 
-                return;
+                return mappings;
             }
 
+            if (mappings == null)
+                mappings = U.newHashMap(keys.size());
+
             Collection<KeyCacheObject> mappedKeys = F.addIfAbsent(mappings, pick, F.<KeyCacheObject>newSet());
 
             assert mappedKeys != null;
@@ -357,6 +367,8 @@ public final class GridDhtForceKeysFuture<K, V> extends GridCompoundFuture<Objec
                 log.debug("Will not rebalance key (local partition is not MOVING) [cacheName=" + cctx.name() +
                     ", key=" + key + ", part=" + locPart + ", locId=" + cctx.nodeId() + ']');
         }
+
+        return mappings;
     }
 
     /**


[07/42] incubator-ignite git commit: Merge remote-tracking branch 'remotes/origin/ignite-sprint-5' into ignite-883_1

Posted by ag...@apache.org.
Merge remote-tracking branch 'remotes/origin/ignite-sprint-5' into ignite-883_1


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

Branch: refs/heads/ignite-389
Commit: 4184f363c340b35008aff553f426d872797d7ee9
Parents: 873e01b 97d0bc1
Author: sboikov <sb...@gridgain.com>
Authored: Wed Jun 3 18:05:13 2015 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Wed Jun 3 18:05:13 2015 +0300

----------------------------------------------------------------------
 assembly/dependencies-visor-console.xml         |    3 +
 .../ignite/cache/eviction/EvictableEntry.java   |    7 +
 .../ignite/cache/eviction/EvictionPolicy.java   |    2 +
 .../cache/eviction/fifo/FifoEvictionPolicy.java |  117 +-
 .../eviction/fifo/FifoEvictionPolicyMBean.java  |   22 +
 .../cache/eviction/lru/LruEvictionPolicy.java   |  135 ++-
 .../eviction/lru/LruEvictionPolicyMBean.java    |   38 +
 .../eviction/random/RandomEvictionPolicy.java   |   10 +-
 .../eviction/sorted/SortedEvictionPolicy.java   |  141 ++-
 .../sorted/SortedEvictionPolicyMBean.java       |   22 +
 .../apache/ignite/cache/query/QueryMetrics.java |    6 +-
 .../discovery/GridDiscoveryManager.java         |    4 +-
 .../cache/CacheEvictableEntryImpl.java          |   31 +
 .../processors/cache/CacheMetricsImpl.java      |    4 +-
 .../processors/cache/GridCacheMapEntry.java     |   64 +-
 .../cache/distributed/dht/GridDhtGetFuture.java |   11 +-
 .../dht/atomic/GridDhtAtomicCache.java          |   13 +-
 .../GridDhtPartitionsExchangeFuture.java        |   46 +-
 .../dht/preloader/GridDhtPreloader.java         |    9 +-
 .../local/atomic/GridLocalAtomicCache.java      |   25 +-
 .../cache/query/GridCacheQueryAdapter.java      |   12 +-
 .../processors/query/GridQueryProcessor.java    |  305 +++--
 .../spi/discovery/tcp/TcpDiscoverySpi.java      |    2 +-
 .../cache/GridCacheAbstractFullApiSelfTest.java |   27 +
 .../cache/GridCacheAbstractMetricsSelfTest.java |   48 +-
 .../GridCacheConcurrentTxMultiNodeTest.java     |    8 +-
 ...idCacheConfigurationConsistencySelfTest.java |   14 +-
 .../cache/GridCacheMemoryModeSelfTest.java      |   23 +-
 .../processors/cache/GridCacheOffHeapTest.java  |    5 +-
 .../cache/GridCacheReloadSelfTest.java          |    6 +-
 .../cache/IgniteCachePeekModesAbstractTest.java |    5 +-
 ...GridCacheDhtEvictionNearReadersSelfTest.java |   11 +-
 .../dht/GridCacheDhtEvictionSelfTest.java       |   11 +-
 .../dht/IgniteCacheMultiTxLockSelfTest.java     |    6 +-
 .../GridCachePartitionedEvictionSelfTest.java   |   11 +-
 .../GridCachePartitionedFullApiSelfTest.java    |   32 +
 ...ePartitionedMultiThreadedPutGetSelfTest.java |    6 +-
 ...edOffHeapTieredMultiNodeFullApiSelfTest.java |    2 +-
 .../cache/eviction/EvictionAbstractTest.java    | 1056 ++++++++++++++++++
 .../GridCacheBatchEvictUnswapSelfTest.java      |    5 +-
 ...heConcurrentEvictionConsistencySelfTest.java |   82 +-
 .../GridCacheConcurrentEvictionsSelfTest.java   |   29 +-
 .../GridCacheDistributedEvictionsSelfTest.java  |    5 +-
 .../GridCacheEmptyEntriesAbstractSelfTest.java  |   11 +-
 .../eviction/GridCacheEvictionAbstractTest.java |  484 --------
 .../GridCacheEvictionTouchSelfTest.java         |   22 +-
 .../cache/eviction/GridCacheMockEntry.java      |    5 +
 .../fifo/FifoEvictionPolicySelfTest.java        |  262 +++++
 ...ridCacheFifoBatchEvictionPolicySelfTest.java |  384 -------
 .../GridCacheFifoEvictionPolicySelfTest.java    |  372 ------
 .../lru/GridCacheLruEvictionPolicySelfTest.java |  417 -------
 .../GridCacheLruNearEvictionPolicySelfTest.java |  136 ---
 ...heNearOnlyLruNearEvictionPolicySelfTest.java |  168 ---
 .../eviction/lru/LruEvictionPolicySelfTest.java |  353 ++++++
 .../lru/LruNearEvictionPolicySelfTest.java      |  140 +++
 .../LruNearOnlyNearEvictionPolicySelfTest.java  |  172 +++
 .../GridCacheRandomEvictionPolicySelfTest.java  |  258 -----
 .../RandomEvictionPolicyCacheSizeSelfTest.java  |    6 +
 .../random/RandomEvictionPolicySelfTest.java    |  357 ++++++
 ...dCacheSortedBatchEvictionPolicySelfTest.java |  385 -------
 ...acheSortedEvictionPolicyPerformanceTest.java |  135 ---
 .../GridCacheSortedEvictionPolicySelfTest.java  |  373 -------
 .../SortedEvictionPolicyPerformanceTest.java    |  134 +++
 .../sorted/SortedEvictionPolicySelfTest.java    |  266 +++++
 .../loadtests/GridCacheMultiNodeLoadTest.java   |    5 +-
 .../GridCachePartitionedAtomicLongLoadTest.java |    6 +-
 .../swap/GridSwapEvictAllBenchmark.java         |    6 +-
 .../IgniteCacheEvictionSelfTestSuite.java       |   14 +-
 .../GridCacheOffheapIndexEntryEvictTest.java    |  200 ++++
 .../cache/GridCacheOffheapIndexGetSelfTest.java |   18 +-
 .../cache/GridCacheQueryMetricsSelfTest.java    |   84 +-
 .../cache/GridIndexingWithNoopSwapSelfTest.java |    6 +-
 ...QueryMultiThreadedOffHeapTieredSelfTest.java |   37 +
 ...eQueryMultiThreadedOffHeapTiredSelfTest.java |   37 -
 .../IgniteCacheQueryMultiThreadedSelfTest.java  |   11 +-
 .../cache/ttl/CacheTtlAbstractSelfTest.java     |    6 +-
 .../IgniteCacheQuerySelfTestSuite.java          |    2 +-
 .../IgniteCacheWithIndexingTestSuite.java       |    1 +
 scripts/git-apply-patch.sh                      |    8 +-
 scripts/git-format-patch.sh                     |    6 +-
 scripts/git-patch-functions.sh                  |   36 +-
 81 files changed, 4240 insertions(+), 3504 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/4184f363/modules/core/src/main/java/org/apache/ignite/internal/managers/discovery/GridDiscoveryManager.java
----------------------------------------------------------------------

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/4184f363/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/TcpDiscoverySpi.java
----------------------------------------------------------------------


[37/42] incubator-ignite git commit: # ignite-sprint-5 added test for ignite-999

Posted by ag...@apache.org.
# ignite-sprint-5 added test for ignite-999


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

Branch: refs/heads/ignite-389
Commit: 837462f5dade1930a8faaffb466c7bf05f9e06f9
Parents: a5b5ec7
Author: sboikov <sb...@gridgain.com>
Authored: Mon Jun 8 14:55:20 2015 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Mon Jun 8 14:55:20 2015 +0300

----------------------------------------------------------------------
 .../DataStreamerMultinodeCreateCacheTest.java   | 97 ++++++++++++++++++++
 .../ignite/testsuites/IgniteCacheTestSuite.java |  1 +
 2 files changed, 98 insertions(+)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/837462f5/modules/core/src/test/java/org/apache/ignite/internal/processors/datastreamer/DataStreamerMultinodeCreateCacheTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/datastreamer/DataStreamerMultinodeCreateCacheTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/datastreamer/DataStreamerMultinodeCreateCacheTest.java
new file mode 100644
index 0000000..2d19d6f
--- /dev/null
+++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/datastreamer/DataStreamerMultinodeCreateCacheTest.java
@@ -0,0 +1,97 @@
+/*
+ * 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.datastreamer;
+
+import org.apache.ignite.*;
+import org.apache.ignite.configuration.*;
+import org.apache.ignite.internal.*;
+import org.apache.ignite.spi.discovery.tcp.*;
+import org.apache.ignite.spi.discovery.tcp.ipfinder.*;
+import org.apache.ignite.spi.discovery.tcp.ipfinder.vm.*;
+import org.apache.ignite.testframework.*;
+import org.apache.ignite.testframework.junits.common.*;
+
+import java.util.concurrent.*;
+import java.util.concurrent.atomic.*;
+
+/**
+ *
+ */
+public class DataStreamerMultinodeCreateCacheTest extends GridCommonAbstractTest {
+    /** */
+    private static final TcpDiscoveryIpFinder IP_FINDER = new TcpDiscoveryVmIpFinder(true);
+
+    /** {@inheritDoc} */
+    @Override protected IgniteConfiguration getConfiguration(String gridName) throws Exception {
+        IgniteConfiguration cfg = super.getConfiguration(gridName);
+
+        ((TcpDiscoverySpi)cfg.getDiscoverySpi()).setIpFinder(IP_FINDER);
+
+        ((TcpDiscoverySpi)cfg.getDiscoverySpi()).setSocketTimeout(5000);
+        ((TcpDiscoverySpi)cfg.getDiscoverySpi()).setAckTimeout(5000);
+
+        return cfg;
+    }
+
+    /** {@inheritDoc} */
+    @Override protected void afterTest() throws Exception {
+        stopAllGrids();
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testCreateCacheAndStream() throws Exception {
+        final int THREADS = 5;
+
+        startGrids(THREADS);
+
+        final AtomicInteger idx = new AtomicInteger();
+
+        IgniteInternalFuture<?> fut = GridTestUtils.runMultiThreadedAsync(new Callable<Object>() {
+            @Override public Object call() throws Exception {
+                int threadIdx = idx.getAndIncrement();
+
+                long stopTime = System.currentTimeMillis() + 60_000;
+
+                Ignite ignite = grid(threadIdx);
+
+                int iter = 0;
+
+                while (System.currentTimeMillis() < stopTime) {
+                    String cacheName = "cache-" + threadIdx + "-" + iter;
+
+                    try (IgniteCache<Integer, String> cache = ignite.getOrCreateCache(cacheName)) {
+                        try (IgniteDataStreamer<Object, Object> stmr = ignite.dataStreamer(cacheName)) {
+                            ((DataStreamerImpl<Object, Object>)stmr).maxRemapCount(0);
+
+                            for (int i = 0; i < 1000; i++)
+                                stmr.addData(i, i);
+                        }
+                    }
+
+                    iter++;
+                }
+
+                return null;
+            }
+        }, THREADS, "create-cache");
+
+        fut.get(2 * 60_000);
+    }
+}

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/837462f5/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteCacheTestSuite.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteCacheTestSuite.java b/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteCacheTestSuite.java
index 9634e9a..7e4409d 100644
--- a/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteCacheTestSuite.java
+++ b/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteCacheTestSuite.java
@@ -111,6 +111,7 @@ public class IgniteCacheTestSuite extends TestSuite {
         suite.addTestSuite(GridCacheStoreValueBytesSelfTest.class);
         suite.addTestSuite(DataStreamProcessorSelfTest.class);
         suite.addTestSuite(DataStreamerMultiThreadedSelfTest.class);
+        suite.addTestSuite(DataStreamerMultinodeCreateCacheTest.class);
         suite.addTestSuite(DataStreamerImplSelfTest.class);
         suite.addTestSuite(GridCacheEntryMemorySizeSelfTest.class);
         suite.addTestSuite(GridCacheClearAllSelfTest.class);


[21/42] incubator-ignite git commit: # ignite-sprint-5 use consistent topology version in streamer

Posted by ag...@apache.org.
# ignite-sprint-5 use consistent topology version in streamer


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

Branch: refs/heads/ignite-389
Commit: dc1d427fa2f5235a5779058a444e3845946a7e07
Parents: 20e5677
Author: sboikov <sb...@gridgain.com>
Authored: Fri Jun 5 09:54:37 2015 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Fri Jun 5 11:21:36 2015 +0300

----------------------------------------------------------------------
 .../affinity/GridAffinityProcessor.java         | 23 ++++-
 .../datastreamer/DataStreamerImpl.java          | 92 ++++++++++++++------
 .../DataStreamerMultiThreadedSelfTest.java      | 59 +++++++++----
 3 files changed, 129 insertions(+), 45 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/dc1d427f/modules/core/src/main/java/org/apache/ignite/internal/processors/affinity/GridAffinityProcessor.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/affinity/GridAffinityProcessor.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/affinity/GridAffinityProcessor.java
index daa2bc2..aac63c8 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/affinity/GridAffinityProcessor.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/affinity/GridAffinityProcessor.java
@@ -164,14 +164,17 @@ public class GridAffinityProcessor extends GridProcessorAdapter {
      *
      * @param cacheName Cache name.
      * @param key Key to map.
+     * @param topVer Topology version.
      * @return Affinity nodes, primary first.
      * @throws IgniteCheckedException If failed.
      */
-    public <K> List<ClusterNode> mapKeyToPrimaryAndBackups(@Nullable String cacheName, K key) throws IgniteCheckedException {
+    public <K> List<ClusterNode> mapKeyToPrimaryAndBackups(@Nullable String cacheName,
+        K key,
+        AffinityTopologyVersion topVer)
+        throws IgniteCheckedException
+    {
         A.notNull(key, "key");
 
-        AffinityTopologyVersion topVer = ctx.discovery().topologyVersionEx();
-
         AffinityInfo affInfo = affinityCache(cacheName, topVer);
 
         if (affInfo == null)
@@ -181,6 +184,20 @@ public class GridAffinityProcessor extends GridProcessorAdapter {
     }
 
     /**
+     * Map single key to primary and backup nodes.
+     *
+     * @param cacheName Cache name.
+     * @param key Key to map.
+     * @return Affinity nodes, primary first.
+     * @throws IgniteCheckedException If failed.
+     */
+    public <K> List<ClusterNode> mapKeyToPrimaryAndBackups(@Nullable String cacheName, K key)
+        throws IgniteCheckedException
+    {
+        return mapKeyToPrimaryAndBackups(cacheName, key, ctx.discovery().topologyVersionEx());
+    }
+
+    /**
      * Gets affinity key for cache key.
      *
      * @param cacheName Cache name.

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/dc1d427f/modules/core/src/main/java/org/apache/ignite/internal/processors/datastreamer/DataStreamerImpl.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/datastreamer/DataStreamerImpl.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/datastreamer/DataStreamerImpl.java
index d16167a..ed8e573 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/datastreamer/DataStreamerImpl.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/datastreamer/DataStreamerImpl.java
@@ -198,19 +198,14 @@ public class DataStreamerImpl<K, V> implements IgniteDataStreamer<K, V>, Delayed
                 // Remap regular mappings.
                 final Buffer buf = bufMappings.remove(id);
 
+                // Only async notification is possible since
+                // discovery thread may be trapped otherwise.
                 if (buf != null) {
-                    // Only async notification is possible since
-                    // discovery thread may be trapped otherwise.
-                    ctx.closure().callLocalSafe(
-                        new Callable<Object>() {
-                            @Override public Object call() throws Exception {
-                                buf.onNodeLeft();
-
-                                return null;
-                            }
-                        },
-                        true /* system pool */
-                    );
+                    waitAffinityAndRun(new Runnable() {
+                        @Override public void run() {
+                            buf.onNodeLeft();
+                        }
+                    }, discoEvt.topologyVersion(), true);
                 }
             }
         };
@@ -248,6 +243,31 @@ public class DataStreamerImpl<K, V> implements IgniteDataStreamer<K, V>, Delayed
     }
 
     /**
+     * @param c Closure to run.
+     * @param topVer Topology version to wait for.
+     * @param async Async flag.
+     */
+    private void waitAffinityAndRun(final Runnable c, long topVer, boolean async) {
+        AffinityTopologyVersion topVer0 = new AffinityTopologyVersion(topVer, 0);
+
+        IgniteInternalFuture<?> fut = ctx.cache().context().exchange().affinityReadyFuture(topVer0);
+
+        if (fut != null && !fut.isDone()) {
+            fut.listen(new CI1<IgniteInternalFuture<?>>() {
+                @Override public void apply(IgniteInternalFuture<?> fut) {
+                    ctx.closure().runLocalSafe(c, true);
+                }
+            });
+        }
+        else {
+            if (async)
+                ctx.closure().runLocalSafe(c, true);
+            else
+                c.run();
+        }
+    }
+
+    /**
      * @return Cache object context.
      */
     public CacheObjectContext cacheObjectContext() {
@@ -527,6 +547,8 @@ public class DataStreamerImpl<K, V> implements IgniteDataStreamer<K, V>, Delayed
 
         boolean initPda = ctx.deploy().enabled() && jobPda == null;
 
+        AffinityTopologyVersion topVer = ctx.cache().context().exchange().readyAffinityVersion();
+
         for (DataStreamerEntry entry : entries) {
             List<ClusterNode> nodes;
 
@@ -543,7 +565,7 @@ public class DataStreamerImpl<K, V> implements IgniteDataStreamer<K, V>, Delayed
                     initPda = false;
                 }
 
-                nodes = nodes(key);
+                nodes = nodes(key, topVer);
             }
             catch (IgniteCheckedException e) {
                 resFut.onDone(e);
@@ -621,10 +643,10 @@ public class DataStreamerImpl<K, V> implements IgniteDataStreamer<K, V>, Delayed
                 }
             };
 
-            GridFutureAdapter<?> f;
+            final GridFutureAdapter<?> f;
 
             try {
-                f = buf.update(entriesForNode, lsnr);
+                f = buf.update(entriesForNode, topVer, lsnr);
             }
             catch (IgniteInterruptedCheckedException e1) {
                 resFut.onDone(e1);
@@ -633,30 +655,38 @@ public class DataStreamerImpl<K, V> implements IgniteDataStreamer<K, V>, Delayed
             }
 
             if (ctx.discovery().node(nodeId) == null) {
-                if (bufMappings.remove(nodeId, buf))
-                    buf.onNodeLeft();
+                if (bufMappings.remove(nodeId, buf)) {
+                    final Buffer buf0 = buf;
 
-                if (f != null)
-                    f.onDone(new ClusterTopologyCheckedException("Failed to wait for request completion " +
-                        "(node has left): " + nodeId));
+                    waitAffinityAndRun(new Runnable() {
+                        @Override public void run() {
+                            buf0.onNodeLeft();
+
+                            if (f != null)
+                                f.onDone(new ClusterTopologyCheckedException("Failed to wait for request completion " +
+                                    "(node has left): " + nodeId));
+                        }
+                    }, ctx.discovery().topologyVersion(), false);
+                }
             }
         }
     }
 
     /**
      * @param key Key to map.
+     * @param topVer Topology version.
      * @return Nodes to send requests to.
      * @throws IgniteCheckedException If failed.
      */
-    private List<ClusterNode> nodes(KeyCacheObject key) throws IgniteCheckedException {
+    private List<ClusterNode> nodes(KeyCacheObject key, AffinityTopologyVersion topVer) throws IgniteCheckedException {
         GridAffinityProcessor aff = ctx.affinity();
 
         List<ClusterNode> res = null;
 
         if (!allowOverwrite())
-            res = aff.mapKeyToPrimaryAndBackups(cacheName, key);
+            res = aff.mapKeyToPrimaryAndBackups(cacheName, key, topVer);
         else {
-            ClusterNode node = aff.mapKeyToNode(cacheName, key);
+            ClusterNode node = aff.mapKeyToNode(cacheName, key, topVer);
 
             if (node != null)
                 res = Collections.singletonList(node);
@@ -959,11 +989,13 @@ public class DataStreamerImpl<K, V> implements IgniteDataStreamer<K, V>, Delayed
 
         /**
          * @param newEntries Infos.
+         * @param topVer Topology version.
          * @param lsnr Listener for the operation future.
          * @throws IgniteInterruptedCheckedException If failed.
          * @return Future for operation.
          */
         @Nullable GridFutureAdapter<?> update(Iterable<DataStreamerEntry> newEntries,
+            AffinityTopologyVersion topVer,
             IgniteInClosure<IgniteInternalFuture<?>> lsnr) throws IgniteInterruptedCheckedException {
             List<DataStreamerEntry> entries0 = null;
             GridFutureAdapter<Object> curFut0;
@@ -986,7 +1018,7 @@ public class DataStreamerImpl<K, V> implements IgniteDataStreamer<K, V>, Delayed
             }
 
             if (entries0 != null) {
-                submit(entries0, curFut0);
+                submit(entries0, topVer, curFut0);
 
                 if (cancelled)
                     curFut0.onDone(new IgniteCheckedException("Data streamer has been cancelled: " + DataStreamerImpl.this));
@@ -1023,7 +1055,7 @@ public class DataStreamerImpl<K, V> implements IgniteDataStreamer<K, V>, Delayed
             }
 
             if (entries0 != null)
-                submit(entries0, curFut0);
+                submit(entries0, null, curFut0);
 
             // Create compound future for this flush.
             GridCompoundFuture<Object, Object> res = null;
@@ -1068,10 +1100,13 @@ public class DataStreamerImpl<K, V> implements IgniteDataStreamer<K, V>, Delayed
 
         /**
          * @param entries Entries to submit.
+         * @param topVer Topology version.
          * @param curFut Current future.
          * @throws IgniteInterruptedCheckedException If interrupted.
          */
-        private void submit(final Collection<DataStreamerEntry> entries, final GridFutureAdapter<Object> curFut)
+        private void submit(final Collection<DataStreamerEntry> entries,
+            @Nullable AffinityTopologyVersion topVer,
+            final GridFutureAdapter<Object> curFut)
             throws IgniteInterruptedCheckedException {
             assert entries != null;
             assert !entries.isEmpty();
@@ -1160,6 +1195,9 @@ public class DataStreamerImpl<K, V> implements IgniteDataStreamer<K, V>, Delayed
 
                 reqs.put(reqId, (GridFutureAdapter<Object>)fut);
 
+                if (topVer == null)
+                    topVer = ctx.cache().context().exchange().readyAffinityVersion();
+
                 DataStreamerRequest req = new DataStreamerRequest(
                     reqId,
                     topicBytes,
@@ -1174,7 +1212,7 @@ public class DataStreamerImpl<K, V> implements IgniteDataStreamer<K, V>, Delayed
                     dep != null ? dep.participants() : null,
                     dep != null ? dep.classLoaderId() : null,
                     dep == null,
-                    ctx.cache().context().exchange().readyAffinityVersion());
+                    topVer);
 
                 try {
                     ctx.io().send(node, TOPIC_DATASTREAM, req, PUBLIC_POOL);

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/dc1d427f/modules/core/src/test/java/org/apache/ignite/internal/processors/datastreamer/DataStreamerMultiThreadedSelfTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/datastreamer/DataStreamerMultiThreadedSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/datastreamer/DataStreamerMultiThreadedSelfTest.java
index 2382a66..e0092d4 100644
--- a/modules/core/src/test/java/org/apache/ignite/internal/processors/datastreamer/DataStreamerMultiThreadedSelfTest.java
+++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/datastreamer/DataStreamerMultiThreadedSelfTest.java
@@ -41,6 +41,9 @@ public class DataStreamerMultiThreadedSelfTest extends GridCommonAbstractTest {
     /** IP finder. */
     private static final TcpDiscoveryIpFinder IP_FINDER = new TcpDiscoveryVmIpFinder(true);
 
+    /** */
+    private boolean dynamicCache;
+
     /** {@inheritDoc} */
     @Override protected IgniteConfiguration getConfiguration(String gridName) throws Exception {
         IgniteConfiguration cfg = super.getConfiguration(gridName);
@@ -50,13 +53,22 @@ public class DataStreamerMultiThreadedSelfTest extends GridCommonAbstractTest {
 
         cfg.setDiscoverySpi(discoSpi);
 
+        if (!dynamicCache)
+            cfg.setCacheConfiguration(cacheConfiguration());
+
+        return cfg;
+    }
+
+    /**
+     * @return Cache configuration.
+     */
+    private CacheConfiguration cacheConfiguration() {
         CacheConfiguration ccfg = defaultCacheConfiguration();
 
         ccfg.setCacheMode(PARTITIONED);
         ccfg.setBackups(1);
-        cfg.setCacheConfiguration(ccfg);
 
-        return cfg;
+        return ccfg;
     }
 
     /** {@inheritDoc} */
@@ -68,6 +80,22 @@ public class DataStreamerMultiThreadedSelfTest extends GridCommonAbstractTest {
      * @throws Exception If failed.
      */
     public void testStartStopIgnites() throws Exception {
+        startStopIgnites();
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testStartStopIgnitesDynamicCache() throws Exception {
+        dynamicCache = true;
+
+        startStopIgnites();
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    private void startStopIgnites() throws Exception {
         for (int attempt = 0; attempt < 3; ++attempt) {
             log.info("Iteration: " + attempt);
             
@@ -75,28 +103,29 @@ public class DataStreamerMultiThreadedSelfTest extends GridCommonAbstractTest {
 
             Set<IgniteFuture> futs = new HashSet<>();
 
-            IgniteInternalFuture<?> fut;
+            final AtomicInteger igniteId = new AtomicInteger(1);
 
-            try (final DataStreamerImpl dataLdr = (DataStreamerImpl)ignite.dataStreamer(null)) {
-                dataLdr.maxRemapCount(0);
+            IgniteInternalFuture<?> fut = GridTestUtils.runMultiThreadedAsync(new Callable<Object>() {
+                @Override public Object call() throws Exception {
+                    for (int i = 1; i < 5; ++i)
+                        startGrid(igniteId.incrementAndGet());
 
-                final AtomicInteger igniteId = new AtomicInteger(1);
+                    return true;
+                }
+            }, 2, "start-node-thread");
 
-                fut = GridTestUtils.runMultiThreadedAsync(new Callable<Object>() {
-                    @Override public Object call() throws Exception {
-                        for (int i = 1; i < 5; ++i)
-                            startGrid(igniteId.incrementAndGet());
+            if (dynamicCache)
+                ignite.getOrCreateCache(cacheConfiguration());
 
-                        return true;
-                    }
-                }, 2, "start-node-thread");
+            try (final DataStreamerImpl dataLdr = (DataStreamerImpl)ignite.dataStreamer(null)) {
+                dataLdr.maxRemapCount(0);
 
-                Random random = new Random();
+                Random rnd = new Random();
 
                 long endTime = U.currentTimeMillis() + 15_000;
 
                 while (!fut.isDone() && U.currentTimeMillis() < endTime)
-                    futs.add(dataLdr.addData(random.nextInt(100_000), random.nextInt(100_000)));
+                    futs.add(dataLdr.addData(rnd.nextInt(100_000), String.valueOf(rnd.nextInt(100_000))));
             }
 
             for (IgniteFuture f : futs)


[27/42] incubator-ignite git commit: #ignite-990: Add method getAllOutTx to IgniteCache.

Posted by ag...@apache.org.
#ignite-990: Add method getAllOutTx to IgniteCache.


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

Branch: refs/heads/ignite-389
Commit: ab20fd9059c21c65f3b34da67d75fc94098850e3
Parents: 5f06f57
Author: ivasilinets <iv...@gridgain.com>
Authored: Fri Jun 5 19:06:10 2015 +0300
Committer: ivasilinets <iv...@gridgain.com>
Committed: Fri Jun 5 19:06:10 2015 +0300

----------------------------------------------------------------------
 .../java/org/apache/ignite/IgniteCache.java     |  11 ++
 .../processors/cache/GridCacheAdapter.java      |  17 ++-
 .../processors/cache/GridCacheProxyImpl.java    |  14 ++-
 .../processors/cache/IgniteCacheProxy.java      |  23 +++++
 .../processors/cache/IgniteInternalCache.java   |  11 +-
 .../processors/igfs/IgfsMetaManager.java        |   2 +-
 .../cache/GridCacheAbstractFullApiSelfTest.java |  83 +++++++++++++++
 .../GridCacheExAbstractFullApiSelfTest.java     | 103 -------------------
 .../GridCacheExColocatedFullApiSelfTest.java    |  33 ------
 .../near/GridCacheExNearFullApiSelfTest.java    |  39 -------
 .../GridCacheExReplicatedFullApiSelfTest.java   |  33 ------
 .../local/GridCacheExLocalFullApiSelfTest.java  |  30 ------
 .../IgniteCacheFullApiSelfTestSuite.java        |   6 --
 13 files changed, 155 insertions(+), 250 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/ab20fd90/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 f9007a2..07f4f3f 100644
--- a/modules/core/src/main/java/org/apache/ignite/IgniteCache.java
+++ b/modules/core/src/main/java/org/apache/ignite/IgniteCache.java
@@ -311,10 +311,21 @@ public interface IgniteCache<K, V> extends javax.cache.Cache<K, V>, IgniteAsyncS
     @IgniteAsyncSupported
     @Override public Map<K, V> getAll(Set<? extends K> keys);
 
+    /**
+     * Gets values from cache. Will bypass started transaction, if any, i.e. will not enlist entries
+     * and will not lock any keys if pessimistic transaction is started by thread.
+     *
+     * @param keys The keys whose associated values are to be returned.
+     * @return A map of entries that were found for the given keys.
+     */
+    @IgniteAsyncSupported
+    public Map<K, V> getAllOutTx(Set<? extends K> keys);
+
     /** {@inheritDoc} */
     @IgniteAsyncSupported
     @Override public boolean containsKey(K key);
 
+    /** {@inheritDoc} */
     @IgniteAsyncSupported
     public boolean containsKeys(Set<? extends K> keys);
 

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/ab20fd90/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 9d98ce7..56b5c15 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
@@ -1220,11 +1220,22 @@ public abstract class GridCacheAdapter<K, V> implements IgniteInternalCache<K, V
     }
 
     /** {@inheritDoc} */
-    @Nullable @Override public Map<K, V> getAllOutTx(List<K> keys) throws IgniteCheckedException {
+    @Nullable @Override public Map<K, V> getAllOutTx(Set<? extends K> keys) throws IgniteCheckedException {
+        return getAllOutTxAsync(keys).get();
+    }
+
+    /** {@inheritDoc} */
+    @Override public IgniteInternalFuture<Map<K, V>> getAllOutTxAsync(Set<? extends K> keys) {
         String taskName = ctx.kernalContext().job().currentTaskName();
 
-        return getAllAsync(keys, !ctx.config().isReadFromBackup(), /*skip tx*/true, null, null, taskName, true, false)
-            .get();
+        return getAllAsync(keys,
+            !ctx.config().isReadFromBackup(),
+            /*skip tx*/true,
+            null,
+            null,
+            taskName,
+            !ctx.keepPortable(),
+            false);
     }
 
     /**

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/ab20fd90/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 9a6d08a..63ba242 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
@@ -329,7 +329,7 @@ public class GridCacheProxyImpl<K, V> implements IgniteInternalCache<K, V>, Exte
     }
 
     /** {@inheritDoc} */
-    @Nullable @Override public Map<K, V> getAllOutTx(List<K> keys) throws IgniteCheckedException {
+    @Nullable @Override public Map<K, V> getAllOutTx(Set<? extends K> keys) throws IgniteCheckedException {
         CacheOperationContext prev = gate.enter(opCtx);
 
         try {
@@ -341,6 +341,18 @@ public class GridCacheProxyImpl<K, V> implements IgniteInternalCache<K, V>, Exte
     }
 
     /** {@inheritDoc} */
+    @Nullable @Override public IgniteInternalFuture<Map<K, V>> getAllOutTxAsync(Set<? extends K> keys) {
+        CacheOperationContext prev = gate.enter(opCtx);
+
+        try {
+            return delegate.getAllOutTxAsync(keys);
+        }
+        finally {
+            gate.leave(prev);
+        }
+    }
+
+    /** {@inheritDoc} */
     @Override public boolean isIgfsDataCache() {
         CacheOperationContext prev = gate.enter(opCtx);
 

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/ab20fd90/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 f840015..4390993 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
@@ -699,6 +699,29 @@ public class IgniteCacheProxy<K, V> extends AsyncSupportAdapter<IgniteCache<K, V
         }
     }
 
+    /** {@inheritDoc} */
+    @Override public Map<K, V> getAllOutTx(Set<? extends K> keys) {
+        try {
+            CacheOperationContext prev = onEnter(opCtx);
+
+            try {
+                if (isAsync()) {
+                    setFuture(delegate.getAllOutTxAsync(keys));
+
+                    return null;
+                }
+                else
+                    return delegate.getAllOutTx(keys);
+            }
+            finally {
+                onLeave(prev);
+            }
+        }
+        catch (IgniteCheckedException e) {
+            throw cacheException(e);
+        }
+    }
+
     /**
      * @param keys Keys.
      * @return Values map.

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/ab20fd90/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/IgniteInternalCache.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/IgniteInternalCache.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/IgniteInternalCache.java
index ccce1b7..d98379c 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/IgniteInternalCache.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/IgniteInternalCache.java
@@ -1623,7 +1623,16 @@ public interface IgniteInternalCache<K, V> extends Iterable<Cache.Entry<K, V>> {
      * @return Value.
      * @throws IgniteCheckedException If failed.
      */
-    @Nullable public Map<K, V> getAllOutTx(List<K> keys) throws IgniteCheckedException;
+    public Map<K, V> getAllOutTx(Set<? extends K> keys) throws IgniteCheckedException;
+
+    /**
+     * Gets values from cache. Will bypass started transaction, if any, i.e. will not enlist entries
+     * and will not lock any keys if pessimistic transaction is started by thread.
+     *
+     * @param keys Keys to get values for.
+     * @return Future for getAllOutTx operation.
+     */
+    public IgniteInternalFuture<Map<K, V>> getAllOutTxAsync(Set<? extends K> keys);
 
     /**
      * Checks whether this cache is IGFS data cache.

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/ab20fd90/modules/core/src/main/java/org/apache/ignite/internal/processors/igfs/IgfsMetaManager.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/igfs/IgfsMetaManager.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/igfs/IgfsMetaManager.java
index e33e0d4..b98c5d8 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/igfs/IgfsMetaManager.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/igfs/IgfsMetaManager.java
@@ -669,7 +669,7 @@ public class IgfsMetaManager extends IgfsManager {
     private Map<String, IgfsListingEntry> directoryListing(IgniteUuid fileId, boolean skipTx) throws IgniteCheckedException {
         assert fileId != null;
 
-        IgfsFileInfo info = skipTx ? id2InfoPrj.getAllOutTx(Collections.singletonList(fileId)).get(fileId) :
+        IgfsFileInfo info = skipTx ? id2InfoPrj.getAllOutTx(Collections.singleton(fileId)).get(fileId) :
             id2InfoPrj.get(fileId);
 
         return info == null ? Collections.<String, IgfsListingEntry>emptyMap() : info.listing();

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/ab20fd90/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAbstractFullApiSelfTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAbstractFullApiSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAbstractFullApiSelfTest.java
index da20869..9bfbd15 100644
--- a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAbstractFullApiSelfTest.java
+++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAbstractFullApiSelfTest.java
@@ -4893,6 +4893,89 @@ public abstract class GridCacheAbstractFullApiSelfTest extends GridCacheAbstract
     }
 
     /**
+     * @throws Exception If failed.
+     */
+    public void testGetOutTx() throws Exception {
+        checkGetOutTx(false);
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testGetOutTxAsync() throws Exception {
+        checkGetOutTx(true);
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    private void checkGetOutTx(boolean async) throws Exception {
+        final AtomicInteger lockEvtCnt = new AtomicInteger();
+
+        IgnitePredicate<Event> lsnr = new IgnitePredicate<Event>() {
+            @Override public boolean apply(Event evt) {
+                lockEvtCnt.incrementAndGet();
+
+                return true;
+            }
+        };
+
+        try {
+            IgniteCache<String, Integer> cache = grid(0).cache(null);
+
+            List<String> keys = primaryKeysForCache(cache, 2);
+
+            assertEquals(2, keys.size());
+
+            cache.put(keys.get(0), 0);
+            cache.put(keys.get(1), 1);
+
+            grid(0).events().localListen(lsnr, EVT_CACHE_OBJECT_LOCKED, EVT_CACHE_OBJECT_UNLOCKED);
+
+            if (async)
+                cache = cache.withAsync();
+
+            try (Transaction tx = transactions().txStart(PESSIMISTIC, REPEATABLE_READ)) {
+                Integer val0 = cache.get(keys.get(0));
+
+                if (async)
+                    val0 = cache.<Integer>future().get();
+
+                assertEquals(0, val0.intValue());
+
+                Map<String, Integer> allOutTx = cache.getAllOutTx(F.asSet(keys.get(1)));
+
+                if (async)
+                    allOutTx = cache.<Map<String, Integer>>future().get();
+
+                assertEquals(1, allOutTx.size());
+
+                assertTrue(allOutTx.containsKey(keys.get(1)));
+
+                assertEquals(1, allOutTx.get(keys.get(1)).intValue());
+            }
+
+            assertTrue(GridTestUtils.waitForCondition(new PA() {
+                @Override public boolean apply() {
+                    info("Lock event count: " + lockEvtCnt.get());
+                    if (atomicityMode() == ATOMIC)
+                        return lockEvtCnt.get() == 0;
+
+                    if (cacheMode() == PARTITIONED && nearEnabled()) {
+                        if (!grid(0).configuration().isClientMode())
+                            return lockEvtCnt.get() == 4;
+                    }
+
+                    return lockEvtCnt.get() == 2;
+                }
+            }, 15000));
+        }
+        finally {
+            grid(0).events().stopLocalListen(lsnr, EVT_CACHE_OBJECT_LOCKED, EVT_CACHE_OBJECT_UNLOCKED);
+        }
+    }
+
+    /**
      * Sets given value, returns old value.
      */
     public static final class SetValueProcessor implements EntryProcessor<String, Integer, Integer> {

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/ab20fd90/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheExAbstractFullApiSelfTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheExAbstractFullApiSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheExAbstractFullApiSelfTest.java
deleted file mode 100644
index 7a1b7cc..0000000
--- a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheExAbstractFullApiSelfTest.java
+++ /dev/null
@@ -1,103 +0,0 @@
-/*
- * Licensed to the Apache Software Foundation (ASF) under one or more
- * contributor license agreements.  See the NOTICE file distributed with
- * this work for additional information regarding copyright ownership.
- * The ASF licenses this file to You under the Apache License, Version 2.0
- * (the "License"); you may not use this file except in compliance with
- * the License.  You may obtain a copy of the License at
- *
- *      http://www.apache.org/licenses/LICENSE-2.0
- *
- * Unless required by applicable law or agreed to in writing, software
- * distributed under the License is distributed on an "AS IS" BASIS,
- * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
- * See the License for the specific language governing permissions and
- * limitations under the License.
- */
-
-package org.apache.ignite.internal.processors.cache;
-
-import org.apache.ignite.configuration.*;
-import org.apache.ignite.events.*;
-import org.apache.ignite.internal.*;
-import org.apache.ignite.internal.util.typedef.*;
-import org.apache.ignite.lang.*;
-import org.apache.ignite.testframework.*;
-import org.apache.ignite.transactions.*;
-
-import java.util.concurrent.atomic.*;
-
-import static org.apache.ignite.events.EventType.*;
-import static org.apache.ignite.transactions.TransactionConcurrency.*;
-import static org.apache.ignite.transactions.TransactionIsolation.*;
-
-/**
- * Abstract test for private cache interface.
- */
-public abstract class GridCacheExAbstractFullApiSelfTest extends GridCacheAbstractSelfTest {
-    /** {@inheritDoc} */
-    @Override protected int gridCount() {
-        return 4;
-    }
-
-    /** {@inheritDoc} */
-    @Override protected NearCacheConfiguration nearConfiguration() {
-        return null;
-    }
-
-    /**
-     * @throws Exception If failed.
-     */
-    public void testGetOutTx() throws Exception {
-        final AtomicInteger lockEvtCnt = new AtomicInteger();
-
-        IgnitePredicate<Event> lsnr = new IgnitePredicate<Event>() {
-            @Override public boolean apply(Event evt) {
-                lockEvtCnt.incrementAndGet();
-
-                return true;
-            }
-        };
-
-        try {
-            grid(0).events().localListen(lsnr, EVT_CACHE_OBJECT_LOCKED, EVT_CACHE_OBJECT_UNLOCKED);
-
-            GridCacheAdapter<String, Integer> cache = ((IgniteKernal)grid(0)).internalCache();
-
-            try (Transaction tx = transactions().txStart(PESSIMISTIC, REPEATABLE_READ)) {
-                int key = 0;
-
-                for (int i = 0; i < 1000; i++) {
-                    if (grid(0).affinity(null).mapKeyToNode("key" + i).id().equals(grid(0).localNode().id())) {
-                        key = i;
-
-                        break;
-                    }
-                }
-
-                cache.get("key" + key);
-
-                for (int i = key + 1; i < 1000; i++) {
-                    if (grid(0).affinity(null).mapKeyToNode("key" + i).id().equals(grid(0).localNode().id())) {
-                        key = i;
-
-                        break;
-                    }
-                }
-
-                cache.getAllOutTx(F.asList("key" + key));
-            }
-
-            assertTrue(GridTestUtils.waitForCondition(new PA() {
-                @Override public boolean apply() {
-                    info("Lock event count: " + lockEvtCnt.get());
-
-                    return lockEvtCnt.get() == (nearEnabled() ? 4 : 2);
-                }
-            }, 15000));
-        }
-        finally {
-            grid(0).events().stopLocalListen(lsnr, EVT_CACHE_OBJECT_LOCKED, EVT_CACHE_OBJECT_UNLOCKED);
-        }
-    }
-}

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/ab20fd90/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/GridCacheExColocatedFullApiSelfTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/GridCacheExColocatedFullApiSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/GridCacheExColocatedFullApiSelfTest.java
deleted file mode 100644
index 55b1f92..0000000
--- a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/dht/GridCacheExColocatedFullApiSelfTest.java
+++ /dev/null
@@ -1,33 +0,0 @@
-/*
- * 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.internal.processors.cache.*;
-
-import static org.apache.ignite.cache.CacheMode.*;
-
-/**
- * Tests private cache interface on colocated cache.
- */
-public class GridCacheExColocatedFullApiSelfTest extends GridCacheExAbstractFullApiSelfTest {
-    /** {@inheritDoc} */
-    @Override protected CacheMode cacheMode() {
-        return PARTITIONED;
-    }
-}

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/ab20fd90/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/near/GridCacheExNearFullApiSelfTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/near/GridCacheExNearFullApiSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/near/GridCacheExNearFullApiSelfTest.java
deleted file mode 100644
index 11ec96a..0000000
--- a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/near/GridCacheExNearFullApiSelfTest.java
+++ /dev/null
@@ -1,39 +0,0 @@
-/*
- * Licensed to the Apache Software Foundation (ASF) under one or more
- * contributor license agreements.  See the NOTICE file distributed with
- * this work for additional information regarding copyright ownership.
- * The ASF licenses this file to You under the Apache License, Version 2.0
- * (the "License"); you may not use this file except in compliance with
- * the License.  You may obtain a copy of the License at
- *
- *      http://www.apache.org/licenses/LICENSE-2.0
- *
- * Unless required by applicable law or agreed to in writing, software
- * distributed under the License is distributed on an "AS IS" BASIS,
- * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
- * See the License for the specific language governing permissions and
- * limitations under the License.
- */
-
-package org.apache.ignite.internal.processors.cache.distributed.near;
-
-import org.apache.ignite.cache.*;
-import org.apache.ignite.configuration.*;
-import org.apache.ignite.internal.processors.cache.*;
-
-import static org.apache.ignite.cache.CacheMode.*;
-
-/**
- * Tests private cache interface on partitioned cache with near enabled.
- */
-public class GridCacheExNearFullApiSelfTest extends GridCacheExAbstractFullApiSelfTest {
-    /** {@inheritDoc} */
-    @Override protected CacheMode cacheMode() {
-        return PARTITIONED;
-    }
-
-    /** {@inheritDoc} */
-    @Override protected NearCacheConfiguration nearConfiguration() {
-        return new NearCacheConfiguration();
-    }
-}

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/ab20fd90/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/replicated/GridCacheExReplicatedFullApiSelfTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/replicated/GridCacheExReplicatedFullApiSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/replicated/GridCacheExReplicatedFullApiSelfTest.java
deleted file mode 100644
index 32ee784..0000000
--- a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/distributed/replicated/GridCacheExReplicatedFullApiSelfTest.java
+++ /dev/null
@@ -1,33 +0,0 @@
-/*
- * 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.replicated;
-
-import org.apache.ignite.cache.*;
-import org.apache.ignite.internal.processors.cache.*;
-
-import static org.apache.ignite.cache.CacheMode.*;
-
-/**
- * Tests private cache interface on replicated cache.
- */
-public class GridCacheExReplicatedFullApiSelfTest extends GridCacheExAbstractFullApiSelfTest {
-    /** {@inheritDoc} */
-    @Override protected CacheMode cacheMode() {
-        return REPLICATED;
-    }
-}

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

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/ab20fd90/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteCacheFullApiSelfTestSuite.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteCacheFullApiSelfTestSuite.java b/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteCacheFullApiSelfTestSuite.java
index 369e041..5fa3283 100644
--- a/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteCacheFullApiSelfTestSuite.java
+++ b/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteCacheFullApiSelfTestSuite.java
@@ -130,12 +130,6 @@ public class IgniteCacheFullApiSelfTestSuite extends TestSuite {
         suite.addTestSuite(GridCachePartitionedNearDisabledOffHeapTieredMultiNodeFullApiSelfTest.class);
         suite.addTestSuite(GridCachePartitionedNearDisabledAtomicOffHeapTieredMultiNodeFullApiSelfTest.class);
 
-        // Private cache API.
-        suite.addTestSuite(GridCacheExLocalFullApiSelfTest.class);
-        suite.addTestSuite(GridCacheExReplicatedFullApiSelfTest.class);
-        suite.addTestSuite(GridCacheExNearFullApiSelfTest.class);
-        suite.addTestSuite(GridCacheExColocatedFullApiSelfTest.class);
-
 //        Multithreaded // TODO: GG-708
 //        suite.addTestSuite(GridCacheLocalFullApiMultithreadedSelfTest.class);
 //        suite.addTestSuite(GridCacheReplicatedFullApiMultithreadedSelfTest.class);


[06/42] incubator-ignite git commit: # ignite-883

Posted by ag...@apache.org.
# ignite-883


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

Branch: refs/heads/ignite-389
Commit: 873e01bbb61ed5782635fa81e1c9834f261f50fe
Parents: 7a2e898
Author: sboikov <sb...@gridgain.com>
Authored: Tue Jun 2 12:29:00 2015 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Wed Jun 3 16:52:47 2015 +0300

----------------------------------------------------------------------
 .../org/apache/ignite/internal/IgnitionEx.java  |  38 +-
 .../internal/managers/GridManagerAdapter.java   |   9 +
 .../checkpoint/GridCheckpointManager.java       |  52 ++-
 .../processors/cache/GridCacheAdapter.java      |   4 +
 .../processors/cache/GridCacheTtlManager.java   |   9 +-
 .../cache/transactions/IgniteTxManager.java     |   3 -
 .../datastructures/DataStructuresProcessor.java |  66 +--
 .../timeout/GridSpiTimeoutObject.java           |  59 +++
 .../timeout/GridTimeoutProcessor.java           |  24 +-
 .../util/nio/GridCommunicationClient.java       |  26 +-
 .../org/apache/ignite/spi/IgniteSpiAdapter.java |  23 +-
 .../org/apache/ignite/spi/IgniteSpiContext.java |  10 +
 .../ignite/spi/IgniteSpiTimeoutObject.java      |  40 ++
 .../spi/checkpoint/noop/NoopCheckpointSpi.java  |   3 +-
 .../communication/tcp/TcpCommunicationSpi.java  | 409 ++++++-------------
 .../ignite/spi/discovery/tcp/ClientImpl.java    |   3 -
 .../ignite/spi/discovery/tcp/ServerImpl.java    |   1 -
 .../spi/discovery/tcp/TcpDiscoverySpi.java      | 156 +------
 .../IgniteCountDownLatchAbstractSelfTest.java   | 102 +++++
 .../IgniteCacheClientNearCacheExpiryTest.java   | 103 +++++
 .../IgniteCacheExpiryPolicyTestSuite.java       |   2 +
 .../testframework/GridSpiTestContext.java       |  10 +
 22 files changed, 642 insertions(+), 510 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/873e01bb/modules/core/src/main/java/org/apache/ignite/internal/IgnitionEx.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/IgnitionEx.java b/modules/core/src/main/java/org/apache/ignite/internal/IgnitionEx.java
index ed33365..5cbe377 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/IgnitionEx.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/IgnitionEx.java
@@ -1447,6 +1447,17 @@ public class IgnitionEx {
                 ensureMultiInstanceSupport(myCfg.getSwapSpaceSpi());
             }
 
+            execSvc = new IgniteThreadPoolExecutor(
+                "pub-" + cfg.getGridName(),
+                cfg.getPublicThreadPoolSize(),
+                cfg.getPublicThreadPoolSize(),
+                DFLT_PUBLIC_KEEP_ALIVE_TIME,
+                new LinkedBlockingQueue<Runnable>(DFLT_PUBLIC_THREADPOOL_QUEUE_CAP));
+
+            if (!myCfg.isClientMode())
+                // Pre-start all threads as they are guaranteed to be needed.
+                ((ThreadPoolExecutor)execSvc).prestartAllCoreThreads();
+
             // Note that since we use 'LinkedBlockingQueue', number of
             // maximum threads has no effect.
             sysExecSvc = new IgniteThreadPoolExecutor(
@@ -1456,30 +1467,8 @@ public class IgnitionEx {
                 DFLT_SYSTEM_KEEP_ALIVE_TIME,
                 new LinkedBlockingQueue<Runnable>(DFLT_SYSTEM_THREADPOOL_QUEUE_CAP));
 
-            boolean isClientMode = Boolean.TRUE.equals(myCfg.isClientMode());
-
-            if (isClientMode) {
-                execSvc = new IgniteThreadPoolExecutor(
-                    "pub-" + cfg.getGridName(),
-                    0,
-                    cfg.getPublicThreadPoolSize(),
-                    2000,
-                    new LinkedBlockingQueue<Runnable>(DFLT_PUBLIC_THREADPOOL_QUEUE_CAP));
-            }
-            else {
-                execSvc = new IgniteThreadPoolExecutor(
-                    "pub-" + cfg.getGridName(),
-                    cfg.getPublicThreadPoolSize(),
-                    cfg.getPublicThreadPoolSize(),
-                    DFLT_PUBLIC_KEEP_ALIVE_TIME,
-                    new LinkedBlockingQueue<Runnable>(DFLT_PUBLIC_THREADPOOL_QUEUE_CAP));
-
-                // Pre-start all threads as they are guaranteed to be needed.
-                ((ThreadPoolExecutor) execSvc).prestartAllCoreThreads();
-
-                // Pre-start all threads as they are guaranteed to be needed.
-                ((ThreadPoolExecutor) sysExecSvc).prestartAllCoreThreads();
-            }
+            // Pre-start all threads as they are guaranteed to be needed.
+            ((ThreadPoolExecutor)sysExecSvc).prestartAllCoreThreads();
 
             // Note that since we use 'LinkedBlockingQueue', number of
             // maximum threads has no effect.
@@ -2007,7 +1996,6 @@ public class IgnitionEx {
             ccfg.setWriteSynchronizationMode(FULL_SYNC);
             ccfg.setCacheMode(cfg.getCacheMode());
             ccfg.setNodeFilter(CacheConfiguration.ALL_NODES);
-            ccfg.setNearConfiguration(new NearCacheConfiguration());
 
             if (cfg.getCacheMode() == PARTITIONED)
                 ccfg.setBackups(cfg.getBackups());

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/873e01bb/modules/core/src/main/java/org/apache/ignite/internal/managers/GridManagerAdapter.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/managers/GridManagerAdapter.java b/modules/core/src/main/java/org/apache/ignite/internal/managers/GridManagerAdapter.java
index 1eb7143..bea4256 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/managers/GridManagerAdapter.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/managers/GridManagerAdapter.java
@@ -23,6 +23,7 @@ import org.apache.ignite.events.*;
 import org.apache.ignite.internal.*;
 import org.apache.ignite.internal.managers.communication.*;
 import org.apache.ignite.internal.managers.eventstorage.*;
+import org.apache.ignite.internal.processors.timeout.*;
 import org.apache.ignite.internal.util.tostring.*;
 import org.apache.ignite.internal.util.typedef.*;
 import org.apache.ignite.internal.util.typedef.internal.*;
@@ -483,6 +484,14 @@ public abstract class GridManagerAdapter<T extends IgniteSpi> implements GridMan
                         return ctx.discovery().tryFailNode(nodeId);
                     }
 
+                    @Override public void addTimeoutObject(IgniteSpiTimeoutObject obj) {
+                        ctx.timeout().addTimeoutObject(new GridSpiTimeoutObject(obj));
+                    }
+
+                    @Override public void removeTimeoutObject(IgniteSpiTimeoutObject obj) {
+                        ctx.timeout().removeTimeoutObject(new GridSpiTimeoutObject(obj));
+                    }
+
                     /**
                      * @param e Exception to handle.
                      * @return GridSpiException Converted exception.

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/873e01bb/modules/core/src/main/java/org/apache/ignite/internal/managers/checkpoint/GridCheckpointManager.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/managers/checkpoint/GridCheckpointManager.java b/modules/core/src/main/java/org/apache/ignite/internal/managers/checkpoint/GridCheckpointManager.java
index 2e80b6f..ce2a36c 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/managers/checkpoint/GridCheckpointManager.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/managers/checkpoint/GridCheckpointManager.java
@@ -56,11 +56,10 @@ public class GridCheckpointManager extends GridManagerAdapter<CheckpointSpi> {
     private final GridMessageListener lsnr = new CheckpointRequestListener();
 
     /** */
-    private final ConcurrentMap<IgniteUuid, CheckpointSet> keyMap = new ConcurrentHashMap8<>();
+    private final ConcurrentMap<IgniteUuid, CheckpointSet> keyMap;
 
     /** */
-    private final Collection<IgniteUuid> closedSess = new GridBoundedConcurrentLinkedHashSet<>(
-        MAX_CLOSED_SESS, MAX_CLOSED_SESS, 0.75f, 256, PER_SEGMENT_Q);
+    private final Collection<IgniteUuid> closedSess;
 
     /** Grid marshaller. */
     private final Marshaller marsh;
@@ -72,6 +71,21 @@ public class GridCheckpointManager extends GridManagerAdapter<CheckpointSpi> {
         super(ctx, ctx.config().getCheckpointSpi());
 
         marsh = ctx.config().getMarshaller();
+
+        if (enabled()) {
+            keyMap = new ConcurrentHashMap8<>();
+
+            closedSess = new GridBoundedConcurrentLinkedHashSet<>(MAX_CLOSED_SESS,
+                MAX_CLOSED_SESS,
+                0.75f,
+                256,
+                PER_SEGMENT_Q);
+        }
+        else {
+            keyMap = null;
+
+            closedSess = null;
+        }
     }
 
     /** {@inheritDoc} */
@@ -112,7 +126,7 @@ public class GridCheckpointManager extends GridManagerAdapter<CheckpointSpi> {
      * @return Session IDs.
      */
     public Collection<IgniteUuid> sessionIds() {
-        return new ArrayList<>(keyMap.keySet());
+        return enabled() ? new ArrayList<>(keyMap.keySet()) : Collections.<IgniteUuid>emptyList();
     }
 
     /**
@@ -125,8 +139,17 @@ public class GridCheckpointManager extends GridManagerAdapter<CheckpointSpi> {
      * @return {@code true} if checkpoint has been actually saved, {@code false} otherwise.
      * @throws IgniteCheckedException Thrown in case of any errors.
      */
-    public boolean storeCheckpoint(GridTaskSessionInternal ses, String key, Object state, ComputeTaskSessionScope scope,
-        long timeout, boolean override) throws IgniteCheckedException {
+    public boolean storeCheckpoint(GridTaskSessionInternal ses,
+        String key,
+        Object state,
+        ComputeTaskSessionScope scope,
+        long timeout,
+        boolean override)
+        throws IgniteCheckedException
+    {
+        if (!enabled())
+            return false;
+
         assert ses != null;
         assert key != null;
 
@@ -239,6 +262,9 @@ public class GridCheckpointManager extends GridManagerAdapter<CheckpointSpi> {
      * @return Whether or not checkpoint was removed.
      */
     public boolean removeCheckpoint(String key) {
+        if (!enabled())
+            return false;
+
         assert key != null;
 
         boolean rmv = false;
@@ -256,6 +282,9 @@ public class GridCheckpointManager extends GridManagerAdapter<CheckpointSpi> {
      * @return Whether or not checkpoint was removed.
      */
     public boolean removeCheckpoint(GridTaskSessionInternal ses, String key) {
+        if (!enabled())
+            return false;
+
         assert ses != null;
         assert key != null;
 
@@ -283,6 +312,9 @@ public class GridCheckpointManager extends GridManagerAdapter<CheckpointSpi> {
      * @throws IgniteCheckedException Thrown in case of any errors.
      */
     @Nullable public Serializable loadCheckpoint(GridTaskSessionInternal ses, String key) throws IgniteCheckedException {
+        if (!enabled())
+            return null;
+
         assert ses != null;
         assert key != null;
 
@@ -309,6 +341,9 @@ public class GridCheckpointManager extends GridManagerAdapter<CheckpointSpi> {
      * @param cleanup Whether cleanup or not.
      */
     public void onSessionEnd(GridTaskSessionInternal ses, boolean cleanup) {
+        if (!enabled())
+            return;
+
         closedSess.add(ses.getId());
 
         // If on task node.
@@ -358,7 +393,7 @@ public class GridCheckpointManager extends GridManagerAdapter<CheckpointSpi> {
     @Override public void printMemoryStats() {
         X.println(">>>");
         X.println(">>> Checkpoint manager memory stats [grid=" + ctx.gridName() + ']');
-        X.println(">>>  keyMap: " + keyMap.size());
+        X.println(">>>  keyMap: " + (keyMap != null ? keyMap.size() : 0));
     }
 
     /**
@@ -407,6 +442,9 @@ public class GridCheckpointManager extends GridManagerAdapter<CheckpointSpi> {
             if (log.isDebugEnabled())
                 log.debug("Received checkpoint request: " + req);
 
+            if (!enabled())
+                return;
+
             IgniteUuid sesId = req.getSessionId();
 
             if (closedSess.contains(sesId)) {

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/873e01bb/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 9d98ce7..4216895 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
@@ -395,6 +395,10 @@ public abstract class GridCacheAdapter<K, V> implements IgniteInternalCache<K, V
 
     /** {@inheritDoc} */
     @Override public GridCacheProxyImpl<K, V> withExpiryPolicy(ExpiryPolicy plc) {
+        assert !CU.isUtilityCache(ctx.name());
+        assert !CU.isAtomicsCache(ctx.name());
+        assert !CU.isMarshallerCache(ctx.name());
+
         CacheOperationContext opCtx = new CacheOperationContext(false, null, false, plc);
 
         return new GridCacheProxyImpl<>(ctx, this, opCtx);

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/873e01bb/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheTtlManager.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheTtlManager.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheTtlManager.java
index 5f9049a..9bd6321 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheTtlManager.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheTtlManager.java
@@ -43,7 +43,14 @@ public class GridCacheTtlManager extends GridCacheManagerAdapter {
 
     /** {@inheritDoc} */
     @Override protected void start0() throws IgniteCheckedException {
-        if (cctx.kernalContext().isDaemon() || !cctx.config().isEagerTtl())
+        boolean cleanupDisabled = cctx.kernalContext().isDaemon() ||
+            !cctx.config().isEagerTtl() ||
+            CU.isAtomicsCache(cctx.name()) ||
+            CU.isMarshallerCache(cctx.name()) ||
+            CU.isUtilityCache(cctx.name()) ||
+            (cctx.kernalContext().clientNode() && cctx.config().getNearConfiguration() == null);
+
+        if (cleanupDisabled)
             return;
 
         cleanupWorker = new CleanupWorker();

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/873e01bb/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/transactions/IgniteTxManager.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/transactions/IgniteTxManager.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/transactions/IgniteTxManager.java
index 4666cca..b6c77f6 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/transactions/IgniteTxManager.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/transactions/IgniteTxManager.java
@@ -1221,9 +1221,6 @@ public class IgniteTxManager extends GridCacheSharedManagerAdapter {
                 collectPendingVersions(dhtTxLoc);
             }
 
-            // 3.1 Call dataStructures manager.
-            cctx.kernalContext().dataStructures().onTxCommitted(tx);
-
             // 4. Unlock write resources.
             unlockMultiple(tx, tx.writeEntries());
 

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/873e01bb/modules/core/src/main/java/org/apache/ignite/internal/processors/datastructures/DataStructuresProcessor.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/datastructures/DataStructuresProcessor.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/datastructures/DataStructuresProcessor.java
index 72911af..27f6a29 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/datastructures/DataStructuresProcessor.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/datastructures/DataStructuresProcessor.java
@@ -32,6 +32,7 @@ import org.apache.ignite.lang.*;
 import org.jetbrains.annotations.*;
 import org.jsr166.*;
 
+import javax.cache.event.*;
 import javax.cache.processor.*;
 import java.io.*;
 import java.util.*;
@@ -40,7 +41,6 @@ import java.util.concurrent.*;
 import static org.apache.ignite.cache.CacheAtomicWriteOrderMode.*;
 import static org.apache.ignite.cache.CacheRebalanceMode.*;
 import static org.apache.ignite.cache.CacheWriteSynchronizationMode.*;
-import static org.apache.ignite.internal.processors.cache.GridCacheOperation.*;
 import static org.apache.ignite.internal.processors.datastructures.DataStructuresProcessor.DataStructureType.*;
 import static org.apache.ignite.transactions.TransactionConcurrency.*;
 import static org.apache.ignite.transactions.TransactionIsolation.*;
@@ -99,6 +99,9 @@ public final class DataStructuresProcessor extends GridProcessorAdapter {
     /** */
     private IgniteInternalCache<CacheDataStructuresCacheKey, List<CacheCollectionInfo>> utilityDataCache;
 
+    /** */
+    private UUID qryId;
+
     /**
      * @param ctx Context.
      */
@@ -112,7 +115,7 @@ public final class DataStructuresProcessor extends GridProcessorAdapter {
 
     /** {@inheritDoc} */
     @SuppressWarnings("unchecked")
-    @Override public void onKernalStart() {
+    @Override public void onKernalStart() throws IgniteCheckedException {
         if (ctx.config().isDaemon())
             return;
 
@@ -139,10 +142,23 @@ public final class DataStructuresProcessor extends GridProcessorAdapter {
 
             seqView = atomicsCache;
 
-            dsCacheCtx = ctx.cache().internalCache(CU.ATOMICS_CACHE_NAME).context();
+            dsCacheCtx = atomicsCache.context();
+
+            qryId = dsCacheCtx.continuousQueries().executeInternalQuery(new DataStructuresEntryListener(),
+                null,
+                dsCacheCtx.isReplicated() && dsCacheCtx.affinityNode(),
+                false);
         }
     }
 
+    /** {@inheritDoc} */
+    @Override public void onKernalStop(boolean cancel) {
+        super.onKernalStop(cancel);
+
+        if (qryId != null)
+            dsCacheCtx.continuousQueries().cancelInternalQuery(qryId);
+    }
+
     /**
      * Gets a sequence from cache or creates one if it's not cached.
      *
@@ -906,8 +922,7 @@ public final class DataStructuresProcessor extends GridProcessorAdapter {
                 dsCacheCtx.gate().enter();
 
                 try (IgniteInternalTx tx = CU.txStartInternal(dsCacheCtx, dsView, PESSIMISTIC, REPEATABLE_READ)) {
-                    GridCacheCountDownLatchValue val = cast(dsView.get(key),
-                        GridCacheCountDownLatchValue.class);
+                    GridCacheCountDownLatchValue val = cast(dsView.get(key), GridCacheCountDownLatchValue.class);
 
                     // Check that count down hasn't been created in other thread yet.
                     GridCacheCountDownLatchEx latch = cast(dsMap.get(key), GridCacheCountDownLatchEx.class);
@@ -1034,28 +1049,19 @@ public final class DataStructuresProcessor extends GridProcessorAdapter {
     }
 
     /**
-     * Transaction committed callback for transaction manager.
      *
-     * @param tx Committed transaction.
      */
-    public <K, V> void onTxCommitted(IgniteInternalTx tx) {
-        if (dsCacheCtx == null)
-            return;
-
-        if (!dsCacheCtx.isDht() && tx.internal() && (!dsCacheCtx.isColocated() || dsCacheCtx.isReplicated())) {
-            Collection<IgniteTxEntry> entries = tx.writeEntries();
-
-            if (log.isDebugEnabled())
-                log.debug("Committed entries: " + entries);
-
-            for (IgniteTxEntry entry : entries) {
-                // Check updated or created GridCacheInternalKey keys.
-                if ((entry.op() == CREATE || entry.op() == UPDATE) && entry.key().internal()) {
-                    GridCacheInternal key = entry.key().value(entry.context().cacheObjectContext(), false);
-
-                    Object val0 = CU.value(entry.value(), entry.context(), false);
+    private class DataStructuresEntryListener implements CacheEntryUpdatedListener<GridCacheInternalKey, GridCacheInternal> {
+        /** {@inheritDoc} */
+        @Override public void onUpdated(Iterable<CacheEntryEvent<? extends GridCacheInternalKey, ? extends GridCacheInternal>> evts)
+            throws CacheEntryListenerException {
+            for (CacheEntryEvent<? extends GridCacheInternalKey, ? extends GridCacheInternal> evt : evts) {
+                if (evt.getEventType() == EventType.CREATED || evt.getEventType() == EventType.UPDATED) {
+                    GridCacheInternal val0 = evt.getValue();
 
                     if (val0 instanceof GridCacheCountDownLatchValue) {
+                        GridCacheInternalKey key = evt.getKey();
+
                         // Notify latch on changes.
                         GridCacheRemovable latch = dsMap.get(key);
 
@@ -1067,8 +1073,6 @@ public final class DataStructuresProcessor extends GridProcessorAdapter {
                             latch0.onUpdate(val.get());
 
                             if (val.get() == 0 && val.autoDelete()) {
-                                entry.cached().markObsolete(dsCacheCtx.versions().next());
-
                                 dsMap.remove(key);
 
                                 latch.onRemoved();
@@ -1080,11 +1084,12 @@ public final class DataStructuresProcessor extends GridProcessorAdapter {
                                 ", actual=" + latch.getClass() + ", value=" + latch + ']');
                         }
                     }
+
                 }
+                else {
+                    assert evt.getEventType() == EventType.REMOVED : evt;
 
-                // Check deleted GridCacheInternal keys.
-                if (entry.op() == DELETE && entry.key().internal()) {
-                    GridCacheInternal key = entry.key().value(entry.context().cacheObjectContext(), false);
+                    GridCacheInternal key = evt.getKey();
 
                     // Entry's val is null if entry deleted.
                     GridCacheRemovable obj = dsMap.remove(key);
@@ -1094,6 +1099,11 @@ public final class DataStructuresProcessor extends GridProcessorAdapter {
                 }
             }
         }
+
+        /** {@inheritDoc} */
+        @Override public String toString() {
+            return S.toString(DataStructuresEntryListener.class, this);
+        }
     }
 
     /**

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/873e01bb/modules/core/src/main/java/org/apache/ignite/internal/processors/timeout/GridSpiTimeoutObject.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/timeout/GridSpiTimeoutObject.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/timeout/GridSpiTimeoutObject.java
new file mode 100644
index 0000000..82267a2
--- /dev/null
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/timeout/GridSpiTimeoutObject.java
@@ -0,0 +1,59 @@
+/*
+ * 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.timeout;
+
+import org.apache.ignite.internal.util.tostring.*;
+import org.apache.ignite.internal.util.typedef.internal.*;
+import org.apache.ignite.lang.*;
+import org.apache.ignite.spi.*;
+
+/**
+ * Wrapper for {@link IgniteSpiTimeoutObject}.
+ */
+public class GridSpiTimeoutObject implements GridTimeoutObject {
+    /** */
+    @GridToStringInclude
+    private final IgniteSpiTimeoutObject obj;
+
+    /**
+     * @param obj SPI object.
+     */
+    public GridSpiTimeoutObject(IgniteSpiTimeoutObject obj) {
+        this.obj = obj;
+    }
+
+    /** {@inheritDoc} */
+    @Override public void onTimeout() {
+        obj.onTimeout();
+    }
+
+    /** {@inheritDoc} */
+    @Override public IgniteUuid timeoutId() {
+        return obj.id();
+    }
+
+    /** {@inheritDoc} */
+    @Override public long endTime() {
+        return obj.endTime();
+    }
+
+    /** {@inheritDoc} */
+    @Override public final String toString() {
+        return S.toString(GridSpiTimeoutObject.class, this);
+    }
+}

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/873e01bb/modules/core/src/main/java/org/apache/ignite/internal/processors/timeout/GridTimeoutProcessor.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/timeout/GridTimeoutProcessor.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/timeout/GridTimeoutProcessor.java
index e9b7717..e4f370c 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/timeout/GridTimeoutProcessor.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/timeout/GridTimeoutProcessor.java
@@ -21,6 +21,7 @@ import org.apache.ignite.*;
 import org.apache.ignite.internal.*;
 import org.apache.ignite.internal.processors.*;
 import org.apache.ignite.internal.util.*;
+import org.apache.ignite.internal.util.tostring.*;
 import org.apache.ignite.internal.util.typedef.*;
 import org.apache.ignite.internal.util.typedef.internal.*;
 import org.apache.ignite.internal.util.worker.*;
@@ -111,8 +112,8 @@ public class GridTimeoutProcessor extends GridProcessorAdapter {
      * @return Cancelable to cancel task.
      */
     public CancelableTask schedule(Runnable task, long delay, long period) {
-        assert delay >= 0;
-        assert period > 0 || period == -1;
+        assert delay >= 0 : delay;
+        assert period > 0 || period == -1 : period;
 
         CancelableTask obj = new CancelableTask(task, U.currentTimeMillis() + delay, period);
 
@@ -203,7 +204,7 @@ public class GridTimeoutProcessor extends GridProcessorAdapter {
      */
     public class CancelableTask implements GridTimeoutObject, Closeable {
         /** */
-        private final IgniteUuid id = new IgniteUuid();
+        private final IgniteUuid id = IgniteUuid.randomUuid();
 
         /** */
         private long endTime;
@@ -215,12 +216,13 @@ public class GridTimeoutProcessor extends GridProcessorAdapter {
         private volatile boolean cancel;
 
         /** */
+        @GridToStringInclude
         private final Runnable task;
 
         /**
+         * @param task Task to execute.
          * @param firstTime First time.
          * @param period Period.
-         * @param task Task to execute.
          */
         CancelableTask(Runnable task, long firstTime, long period) {
             this.task = task;
@@ -243,19 +245,10 @@ public class GridTimeoutProcessor extends GridProcessorAdapter {
             if (cancel)
                 return;
 
-            long startTime = U.currentTimeMillis();
-
             try {
                 task.run();
             }
             finally {
-                long executionTime = U.currentTimeMillis() - startTime;
-
-                if (executionTime > 10) {
-                    U.warn(log, "Timer task take a lot of time, tasks submitted to GridTimeoutProcessor must work " +
-                        "quickly [executionTime=" + executionTime + ']');
-                }
-
                 if (!cancel && period > 0) {
                     endTime = U.currentTimeMillis() + period;
 
@@ -273,5 +266,10 @@ public class GridTimeoutProcessor extends GridProcessorAdapter {
                 removeTimeoutObject(this);
             }
         }
+
+        /** {@inheritDoc} */
+        @Override public String toString() {
+            return S.toString(CancelableTask.class, this);
+        }
     }
 }

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/873e01bb/modules/core/src/main/java/org/apache/ignite/internal/util/nio/GridCommunicationClient.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/util/nio/GridCommunicationClient.java b/modules/core/src/main/java/org/apache/ignite/internal/util/nio/GridCommunicationClient.java
index 31396fb..2f7fd88 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/util/nio/GridCommunicationClient.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/util/nio/GridCommunicationClient.java
@@ -38,58 +38,58 @@ public interface GridCommunicationClient {
      * @param handshakeC Handshake.
      * @throws IgniteCheckedException If handshake failed.
      */
-    void doHandshake(IgniteInClosure2X<InputStream, OutputStream> handshakeC) throws IgniteCheckedException;
+    public void doHandshake(IgniteInClosure2X<InputStream, OutputStream> handshakeC) throws IgniteCheckedException;
 
     /**
      * @return {@code True} if client has been closed by this call,
      *      {@code false} if failed to close client (due to concurrent reservation or concurrent close).
      */
-    boolean close();
+    public boolean close();
 
     /**
      * Forces client close.
      */
-    void forceClose();
+    public void forceClose();
 
     /**
      * @return {@code True} if client is closed;
      */
-    boolean closed();
+    public boolean closed();
 
     /**
      * @return {@code True} if client was reserved, {@code false} otherwise.
      */
-    boolean reserve();
+    public boolean reserve();
 
     /**
      * Releases this client by decreasing reservations.
      */
-    void release();
+    public void release();
 
     /**
      * @return {@code True} if client was reserved.
      */
-    boolean reserved();
+    public boolean reserved();
 
     /**
      * Gets idle time of this client.
      *
      * @return Idle time of this client.
      */
-    long getIdleTime();
+    public long getIdleTime();
 
     /**
      * @param data Data to send.
      * @throws IgniteCheckedException If failed.
      */
-    void sendMessage(ByteBuffer data) throws IgniteCheckedException;
+    public void sendMessage(ByteBuffer data) throws IgniteCheckedException;
 
     /**
      * @param data Data to send.
      * @param len Length.
      * @throws IgniteCheckedException If failed.
      */
-    void sendMessage(byte[] data, int len) throws IgniteCheckedException;
+    public void sendMessage(byte[] data, int len) throws IgniteCheckedException;
 
     /**
      * @param nodeId Node ID (provided only if versions of local and remote nodes are different).
@@ -97,16 +97,16 @@ public interface GridCommunicationClient {
      * @throws IgniteCheckedException If failed.
      * @return {@code True} if should try to resend message.
      */
-    boolean sendMessage(@Nullable UUID nodeId, Message msg) throws IgniteCheckedException;
+    public boolean sendMessage(@Nullable UUID nodeId, Message msg) throws IgniteCheckedException;
 
     /**
      * @param timeout Timeout.
      * @throws IOException If failed.
      */
-    void flushIfNeeded(long timeout) throws IOException;
+    public void flushIfNeeded(long timeout) throws IOException;
 
     /**
      * @return {@code True} if send is asynchronous.
      */
-    boolean async();
+    public boolean async();
 }

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/873e01bb/modules/core/src/main/java/org/apache/ignite/spi/IgniteSpiAdapter.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/spi/IgniteSpiAdapter.java b/modules/core/src/main/java/org/apache/ignite/spi/IgniteSpiAdapter.java
index d3c0587..c9c633f 100644
--- a/modules/core/src/main/java/org/apache/ignite/spi/IgniteSpiAdapter.java
+++ b/modules/core/src/main/java/org/apache/ignite/spi/IgniteSpiAdapter.java
@@ -23,6 +23,7 @@ import org.apache.ignite.events.*;
 import org.apache.ignite.internal.*;
 import org.apache.ignite.internal.managers.communication.*;
 import org.apache.ignite.internal.managers.eventstorage.*;
+import org.apache.ignite.internal.processors.timeout.*;
 import org.apache.ignite.internal.util.*;
 import org.apache.ignite.internal.util.typedef.*;
 import org.apache.ignite.internal.util.typedef.internal.*;
@@ -542,10 +543,20 @@ public abstract class IgniteSpiAdapter implements IgniteSpi, IgniteSpiManagement
         return U.spiAttribute(this, attrName);
     }
 
+    /** {@inheritDoc} */
+    protected void addTimeoutObject(IgniteSpiTimeoutObject obj) {
+        spiCtx.addTimeoutObject(obj);
+    }
+
+    /** {@inheritDoc} */
+    protected void removeTimeoutObject(IgniteSpiTimeoutObject obj) {
+        spiCtx.removeTimeoutObject(obj);
+    }
+
     /**
      * Temporarily SPI context.
      */
-    private static class GridDummySpiContext implements IgniteSpiContext {
+    private class GridDummySpiContext implements IgniteSpiContext {
         /** */
         private final ClusterNode locNode;
 
@@ -716,5 +727,15 @@ public abstract class IgniteSpiAdapter implements IgniteSpi, IgniteSpiManagement
         @Override public boolean tryFailNode(UUID nodeId) {
             return false;
         }
+
+        /** {@inheritDoc} */
+        @Override public void addTimeoutObject(IgniteSpiTimeoutObject obj) {
+            ((IgniteKernal)ignite).context().timeout().addTimeoutObject(new GridSpiTimeoutObject(obj));
+        }
+
+        /** {@inheritDoc} */
+        @Override public void removeTimeoutObject(IgniteSpiTimeoutObject obj) {
+            ((IgniteKernal)ignite).context().timeout().removeTimeoutObject(new GridSpiTimeoutObject(obj));
+        }
     }
 }

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/873e01bb/modules/core/src/main/java/org/apache/ignite/spi/IgniteSpiContext.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/spi/IgniteSpiContext.java b/modules/core/src/main/java/org/apache/ignite/spi/IgniteSpiContext.java
index 55f46e5..f83326c 100644
--- a/modules/core/src/main/java/org/apache/ignite/spi/IgniteSpiContext.java
+++ b/modules/core/src/main/java/org/apache/ignite/spi/IgniteSpiContext.java
@@ -310,4 +310,14 @@ public interface IgniteSpiContext {
      * @return If node was failed.
      */
     public boolean tryFailNode(UUID nodeId);
+
+    /**
+     * @param c Timeout object.
+     */
+    public void addTimeoutObject(IgniteSpiTimeoutObject c);
+
+    /**
+     * @param c Timeout object.
+     */
+    public void removeTimeoutObject(IgniteSpiTimeoutObject c);
 }

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/873e01bb/modules/core/src/main/java/org/apache/ignite/spi/IgniteSpiTimeoutObject.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/spi/IgniteSpiTimeoutObject.java b/modules/core/src/main/java/org/apache/ignite/spi/IgniteSpiTimeoutObject.java
new file mode 100644
index 0000000..f7fbd27f
--- /dev/null
+++ b/modules/core/src/main/java/org/apache/ignite/spi/IgniteSpiTimeoutObject.java
@@ -0,0 +1,40 @@
+/*
+ * 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.spi;
+
+import org.apache.ignite.lang.*;
+
+/**
+ *
+ */
+public interface IgniteSpiTimeoutObject {
+    /**
+     * @return Unique object ID.
+     */
+    public IgniteUuid id();
+
+    /**
+     * @return End time.
+     */
+    public long endTime();
+
+    /**
+     * Timeout callback.
+     */
+    public void onTimeout();
+}

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/873e01bb/modules/core/src/main/java/org/apache/ignite/spi/checkpoint/noop/NoopCheckpointSpi.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/spi/checkpoint/noop/NoopCheckpointSpi.java b/modules/core/src/main/java/org/apache/ignite/spi/checkpoint/noop/NoopCheckpointSpi.java
index 460cff3..832d872 100644
--- a/modules/core/src/main/java/org/apache/ignite/spi/checkpoint/noop/NoopCheckpointSpi.java
+++ b/modules/core/src/main/java/org/apache/ignite/spi/checkpoint/noop/NoopCheckpointSpi.java
@@ -51,8 +51,7 @@ public class NoopCheckpointSpi extends IgniteSpiAdapter implements CheckpointSpi
     }
 
     /** {@inheritDoc} */
-    @Override
-    public boolean saveCheckpoint(String key, byte[] state, long timeout, boolean overwrite) throws IgniteSpiException {
+    @Override public boolean saveCheckpoint(String key, byte[] state, long timeout, boolean overwrite) {
         return false;
     }
 

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/873e01bb/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 19e54c8..b324ab2 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
@@ -267,7 +267,7 @@ public class TcpCommunicationSpi extends IgniteSpiAdapter
                                                 log.debug("Session was closed but there are unacknowledged messages, " +
                                                     "will try to reconnect [rmtNode=" + recoveryData.node().id() + ']');
 
-                                            recoveryWorker.addReconnectRequest(recoveryData);
+                                            commWorker.addReconnectRequest(recoveryData);
                                         }
                                     }
                                     else
@@ -647,17 +647,11 @@ public class TcpCommunicationSpi extends IgniteSpiAdapter
     /** Socket write timeout. */
     private long sockWriteTimeout = DFLT_SOCK_WRITE_TIMEOUT;
 
-    /** Idle client worker. */
-    private IdleClientWorker idleClientWorker;
-
     /** Flush client worker. */
     private ClientFlushWorker clientFlushWorker;
 
-    /** Socket timeout worker. */
-    private SocketTimeoutWorker sockTimeoutWorker;
-
-    /** Recovery worker. */
-    private RecoveryWorker recoveryWorker;
+    /** Recovery and idle clients handler. */
+    private CommunicationWorker commWorker;
 
     /** Clients. */
     private final ConcurrentMap<UUID, GridCommunicationClient> clients = GridConcurrentFactory.newMap();
@@ -1274,13 +1268,9 @@ public class TcpCommunicationSpi extends IgniteSpiAdapter
 
         nioSrvr.start();
 
-        idleClientWorker = new IdleClientWorker();
-
-        idleClientWorker.start();
-
-        recoveryWorker = new RecoveryWorker();
+        commWorker = new CommunicationWorker();
 
-        recoveryWorker.start();
+        commWorker.start();
 
         if (connBufSize > 0) {
             clientFlushWorker = new ClientFlushWorker();
@@ -1288,10 +1278,6 @@ public class TcpCommunicationSpi extends IgniteSpiAdapter
             clientFlushWorker.start();
         }
 
-        sockTimeoutWorker = new SocketTimeoutWorker();
-
-        sockTimeoutWorker.start();
-
         // Ack start.
         if (log.isDebugEnabled())
             log.debug(startInfo());
@@ -1445,15 +1431,11 @@ public class TcpCommunicationSpi extends IgniteSpiAdapter
         if (nioSrvr != null)
             nioSrvr.stop();
 
-        U.interrupt(idleClientWorker);
         U.interrupt(clientFlushWorker);
-        U.interrupt(sockTimeoutWorker);
-        U.interrupt(recoveryWorker);
+        U.interrupt(commWorker);
 
-        U.join(idleClientWorker, log);
         U.join(clientFlushWorker, log);
-        U.join(sockTimeoutWorker, log);
-        U.join(recoveryWorker, log);
+        U.join(commWorker, log);
 
         // Force closing on stop (safety).
         for (GridCommunicationClient client : clients.values())
@@ -1461,7 +1443,7 @@ public class TcpCommunicationSpi extends IgniteSpiAdapter
 
         // Clear resources.
         nioSrvr = null;
-        idleClientWorker = null;
+        commWorker = null;
 
         boundTcpPort = -1;
 
@@ -1899,7 +1881,7 @@ public class TcpCommunicationSpi extends IgniteSpiAdapter
     ) throws IgniteCheckedException {
         HandshakeTimeoutObject<T> obj = new HandshakeTimeoutObject<>(client, U.currentTimeMillis() + timeout);
 
-        sockTimeoutWorker.addTimeoutObject(obj);
+        addTimeoutObject(obj);
 
         long rcvCnt = 0;
 
@@ -2005,7 +1987,7 @@ public class TcpCommunicationSpi extends IgniteSpiAdapter
             boolean cancelled = obj.cancel();
 
             if (cancelled)
-                sockTimeoutWorker.removeTimeoutObject(obj);
+                removeTimeoutObject(obj);
 
             // Ignoring whatever happened after timeout - reporting only timeout event.
             if (!cancelled)
@@ -2041,15 +2023,11 @@ public class TcpCommunicationSpi extends IgniteSpiAdapter
         if (nioSrvr != null)
             nioSrvr.stop();
 
-        U.interrupt(idleClientWorker);
         U.interrupt(clientFlushWorker);
-        U.interrupt(sockTimeoutWorker);
-        U.interrupt(recoveryWorker);
+        U.interrupt(commWorker);
 
-        U.join(idleClientWorker, log);
         U.join(clientFlushWorker, log);
-        U.join(sockTimeoutWorker, log);
-        U.join(recoveryWorker, log);
+        U.join(commWorker, log);
 
         for (GridCommunicationClient client : clients.values())
             client.forceClose();
@@ -2156,119 +2134,6 @@ public class TcpCommunicationSpi extends IgniteSpiAdapter
     /**
      *
      */
-    private class IdleClientWorker extends IgniteSpiThread {
-        /**
-         *
-         */
-        IdleClientWorker() {
-            super(gridName, "nio-idle-client-collector", log);
-        }
-
-        /** {@inheritDoc} */
-        @SuppressWarnings({"BusyWait"})
-        @Override protected void body() throws InterruptedException {
-            while (!isInterrupted()) {
-                cleanupRecovery();
-
-                for (Map.Entry<UUID, GridCommunicationClient> e : clients.entrySet()) {
-                    UUID nodeId = e.getKey();
-
-                    GridCommunicationClient client = e.getValue();
-
-                    ClusterNode node = getSpiContext().node(nodeId);
-
-                    if (node == null) {
-                        if (log.isDebugEnabled())
-                            log.debug("Forcing close of non-existent node connection: " + nodeId);
-
-                        client.forceClose();
-
-                        clients.remove(nodeId, client);
-
-                        continue;
-                    }
-
-                    GridNioRecoveryDescriptor recovery = null;
-
-                    if (client instanceof GridTcpNioCommunicationClient) {
-                        recovery = recoveryDescs.get(new ClientKey(node.id(), node.order()));
-
-                        if (recovery != null && recovery.lastAcknowledged() != recovery.received()) {
-                            RecoveryLastReceivedMessage msg = new RecoveryLastReceivedMessage(recovery.received());
-
-                            if (log.isDebugEnabled())
-                                log.debug("Send recovery acknowledgement on timeout [rmtNode=" + nodeId +
-                                    ", rcvCnt=" + msg.received() + ']');
-
-                            nioSrvr.sendSystem(((GridTcpNioCommunicationClient)client).session(), msg);
-
-                            recovery.lastAcknowledged(msg.received());
-
-                            continue;
-                        }
-                    }
-
-                    long idleTime = client.getIdleTime();
-
-                    if (idleTime >= idleConnTimeout) {
-                        if (recovery != null &&
-                            recovery.nodeAlive(getSpiContext().node(nodeId)) &&
-                            !recovery.messagesFutures().isEmpty()) {
-                            if (log.isDebugEnabled())
-                                log.debug("Node connection is idle, but there are unacknowledged messages, " +
-                                    "will wait: " + nodeId);
-
-                            continue;
-                        }
-
-                        if (log.isDebugEnabled())
-                            log.debug("Closing idle node connection: " + nodeId);
-
-                        if (client.close() || client.closed())
-                            clients.remove(nodeId, client);
-                    }
-                }
-
-                Thread.sleep(idleConnTimeout);
-            }
-        }
-
-        /**
-         *
-         */
-        private void cleanupRecovery() {
-            Set<ClientKey> left = null;
-
-            for (Map.Entry<ClientKey, GridNioRecoveryDescriptor> e : recoveryDescs.entrySet()) {
-                if (left != null && left.contains(e.getKey()))
-                    continue;
-
-                GridNioRecoveryDescriptor recoverySnd = e.getValue();
-
-                if (!recoverySnd.nodeAlive(getSpiContext().node(recoverySnd.node().id()))) {
-                    if (left == null)
-                        left = new HashSet<>();
-
-                    left.add(e.getKey());
-                }
-            }
-
-            if (left != null) {
-                assert !left.isEmpty();
-
-                for (ClientKey id : left) {
-                    GridNioRecoveryDescriptor recoverySnd = recoveryDescs.remove(id);
-
-                    if (recoverySnd != null)
-                        recoverySnd.onNodeLeft();
-                }
-            }
-        }
-    }
-
-    /**
-     *
-     */
     private class ClientFlushWorker extends IgniteSpiThread {
         /**
          *
@@ -2319,157 +2184,164 @@ public class TcpCommunicationSpi extends IgniteSpiAdapter
     }
 
     /**
-     * Handles sockets timeouts.
+     *
      */
-    private class SocketTimeoutWorker extends IgniteSpiThread {
-        /** Time-based sorted set for timeout objects. */
-        private final GridConcurrentSkipListSet<HandshakeTimeoutObject> timeoutObjs =
-            new GridConcurrentSkipListSet<>(new Comparator<HandshakeTimeoutObject>() {
-                @Override public int compare(HandshakeTimeoutObject o1, HandshakeTimeoutObject o2) {
-                    long time1 = o1.endTime();
-                    long time2 = o2.endTime();
-
-                    long id1 = o1.id();
-                    long id2 = o2.id();
-
-                    return time1 < time2 ? -1 : time1 > time2 ? 1 :
-                        id1 < id2 ? -1 : id1 > id2 ? 1 : 0;
-                }
-            });
-
-        /** Mutex. */
-        private final Object mux0 = new Object();
+    private class CommunicationWorker extends IgniteSpiThread {
+        /** */
+        private final BlockingQueue<GridNioRecoveryDescriptor> q = new LinkedBlockingQueue<>();
 
         /**
          *
          */
-        SocketTimeoutWorker() {
-            super(gridName, "tcp-comm-sock-timeout-worker", log);
+        private CommunicationWorker() {
+            super(gridName, "tcp-comm-worker", log);
         }
 
-        /**
-         * @param timeoutObj Timeout object to add.
-         */
-        @SuppressWarnings({"NakedNotify"})
-        public void addTimeoutObject(HandshakeTimeoutObject timeoutObj) {
-            assert timeoutObj != null && timeoutObj.endTime() > 0 && timeoutObj.endTime() != Long.MAX_VALUE;
+        /** {@inheritDoc} */
+        @Override protected void body() throws InterruptedException {
+            if (log.isDebugEnabled())
+                log.debug("Tcp communication worker has been started.");
 
-            timeoutObjs.add(timeoutObj);
+            while (!isInterrupted()) {
+                GridNioRecoveryDescriptor recoveryDesc = q.poll(idleConnTimeout, TimeUnit.MILLISECONDS);
 
-            if (timeoutObjs.firstx() == timeoutObj) {
-                synchronized (mux0) {
-                    mux0.notifyAll();
-                }
+                if (recoveryDesc != null)
+                    processRecovery(recoveryDesc);
+                else
+                    processIdle();
             }
         }
 
         /**
-         * @param timeoutObj Timeout object to remove.
+         *
          */
-        public void removeTimeoutObject(HandshakeTimeoutObject timeoutObj) {
-            assert timeoutObj != null;
+        private void processIdle() {
+            cleanupRecovery();
 
-            timeoutObjs.remove(timeoutObj);
-        }
+            for (Map.Entry<UUID, GridCommunicationClient> e : clients.entrySet()) {
+                UUID nodeId = e.getKey();
 
-        /** {@inheritDoc} */
-        @Override protected void body() throws InterruptedException {
-            if (log.isDebugEnabled())
-                log.debug("Socket timeout worker has been started.");
+                GridCommunicationClient client = e.getValue();
 
-            while (!isInterrupted()) {
-                long now = U.currentTimeMillis();
+                ClusterNode node = getSpiContext().node(nodeId);
 
-                for (Iterator<HandshakeTimeoutObject> iter = timeoutObjs.iterator(); iter.hasNext(); ) {
-                    HandshakeTimeoutObject timeoutObj = iter.next();
+                if (node == null) {
+                    if (log.isDebugEnabled())
+                        log.debug("Forcing close of non-existent node connection: " + nodeId);
+
+                    client.forceClose();
+
+                    clients.remove(nodeId, client);
+
+                    continue;
+                }
 
-                    if (timeoutObj.endTime() <= now) {
-                        iter.remove();
+                GridNioRecoveryDescriptor recovery = null;
 
-                        timeoutObj.onTimeout();
+                if (client instanceof GridTcpNioCommunicationClient) {
+                    recovery = recoveryDescs.get(new ClientKey(node.id(), node.order()));
+
+                    if (recovery != null && recovery.lastAcknowledged() != recovery.received()) {
+                        RecoveryLastReceivedMessage msg = new RecoveryLastReceivedMessage(recovery.received());
+
+                        if (log.isDebugEnabled())
+                            log.debug("Send recovery acknowledgement on timeout [rmtNode=" + nodeId +
+                                ", rcvCnt=" + msg.received() + ']');
+
+                        nioSrvr.sendSystem(((GridTcpNioCommunicationClient)client).session(), msg);
+
+                        recovery.lastAcknowledged(msg.received());
+
+                        continue;
                     }
-                    else
-                        break;
                 }
 
-                synchronized (mux0) {
-                    while (true) {
-                        // Access of the first element must be inside of
-                        // synchronization block, so we don't miss out
-                        // on thread notification events sent from
-                        // 'addTimeoutObject(..)' method.
-                        HandshakeTimeoutObject first = timeoutObjs.firstx();
+                long idleTime = client.getIdleTime();
 
-                        if (first != null) {
-                            long waitTime = first.endTime() - U.currentTimeMillis();
+                if (idleTime >= idleConnTimeout) {
+                    if (recovery != null &&
+                        recovery.nodeAlive(getSpiContext().node(nodeId)) &&
+                        !recovery.messagesFutures().isEmpty()) {
+                        if (log.isDebugEnabled())
+                            log.debug("Node connection is idle, but there are unacknowledged messages, " +
+                                "will wait: " + nodeId);
 
-                            if (waitTime > 0)
-                                mux0.wait(waitTime);
-                            else
-                                break;
-                        }
-                        else
-                            mux0.wait(5000);
+                        continue;
                     }
+
+                    if (log.isDebugEnabled())
+                        log.debug("Closing idle node connection: " + nodeId);
+
+                    if (client.close() || client.closed())
+                        clients.remove(nodeId, client);
                 }
             }
         }
-    }
-
-    /**
-     *
-     */
-    private class RecoveryWorker extends IgniteSpiThread {
-        /** */
-        private final BlockingQueue<GridNioRecoveryDescriptor> q = new LinkedBlockingQueue<>();
 
         /**
          *
          */
-        private RecoveryWorker() {
-            super(gridName, "tcp-comm-recovery-worker", log);
-        }
-
-        /** {@inheritDoc} */
-        @Override protected void body() throws InterruptedException {
-            if (log.isDebugEnabled())
-                log.debug("Recovery worker has been started.");
+        private void cleanupRecovery() {
+            Set<ClientKey> left = null;
 
-            while (!isInterrupted()) {
-                GridNioRecoveryDescriptor recoveryDesc = q.take();
+            for (Map.Entry<ClientKey, GridNioRecoveryDescriptor> e : recoveryDescs.entrySet()) {
+                if (left != null && left.contains(e.getKey()))
+                    continue;
 
-                assert recoveryDesc != null;
+                GridNioRecoveryDescriptor recoverySnd = e.getValue();
 
-                ClusterNode node = recoveryDesc.node();
+                if (!recoverySnd.nodeAlive(getSpiContext().node(recoverySnd.node().id()))) {
+                    if (left == null)
+                        left = new HashSet<>();
 
-                if (clients.containsKey(node.id()) || !recoveryDesc.nodeAlive(getSpiContext().node(node.id())))
-                    continue;
+                    left.add(e.getKey());
+                }
+            }
 
-                try {
-                    if (log.isDebugEnabled())
-                        log.debug("Recovery reconnect [rmtNode=" + recoveryDesc.node().id() + ']');
+            if (left != null) {
+                assert !left.isEmpty();
 
-                    GridCommunicationClient client = reserveClient(node);
+                for (ClientKey id : left) {
+                    GridNioRecoveryDescriptor recoverySnd = recoveryDescs.remove(id);
 
-                    client.release();
+                    if (recoverySnd != null)
+                        recoverySnd.onNodeLeft();
                 }
-                catch (IgniteCheckedException | IgniteException e) {
-                    if (recoveryDesc.nodeAlive(getSpiContext().node(node.id()))) {
-                        if (log.isDebugEnabled())
-                            log.debug("Recovery reconnect failed, will retry " +
-                                "[rmtNode=" + recoveryDesc.node().id() + ", err=" + e + ']');
+            }
+        }
 
-                        addReconnectRequest(recoveryDesc);
-                    }
-                    else {
-                        if (log.isDebugEnabled())
-                            log.debug("Recovery reconnect failed, " +
-                                "node left [rmtNode=" + recoveryDesc.node().id() + ", err=" + e + ']');
+        /**
+         * @param recoveryDesc Recovery descriptor.
+         */
+        private void processRecovery(GridNioRecoveryDescriptor recoveryDesc) {
+            ClusterNode node = recoveryDesc.node();
 
-                        onException("Recovery reconnect failed, node left [rmtNode=" + recoveryDesc.node().id() + "]",
-                            e);
-                    }
+            if (clients.containsKey(node.id()) || !recoveryDesc.nodeAlive(getSpiContext().node(node.id())))
+                return;
 
+            try {
+                if (log.isDebugEnabled())
+                    log.debug("Recovery reconnect [rmtNode=" + recoveryDesc.node().id() + ']');
+
+                GridCommunicationClient client = reserveClient(node);
+
+                client.release();
+            }
+            catch (IgniteCheckedException | IgniteException e) {
+                if (recoveryDesc.nodeAlive(getSpiContext().node(node.id()))) {
+                    if (log.isDebugEnabled())
+                        log.debug("Recovery reconnect failed, will retry " +
+                            "[rmtNode=" + recoveryDesc.node().id() + ", err=" + e + ']');
+
+                    addReconnectRequest(recoveryDesc);
+                }
+                else {
+                    if (log.isDebugEnabled())
+                        log.debug("Recovery reconnect failed, " +
+                            "node left [rmtNode=" + recoveryDesc.node().id() + ", err=" + e + ']');
+
+                    onException("Recovery reconnect failed, node left [rmtNode=" + recoveryDesc.node().id() + "]",
+                        e);
                 }
             }
         }
@@ -2497,12 +2369,9 @@ public class TcpCommunicationSpi extends IgniteSpiAdapter
     /**
      *
      */
-    private static class HandshakeTimeoutObject<T> {
-        /** */
-        private static final AtomicLong idGen = new AtomicLong();
-
+    private static class HandshakeTimeoutObject<T> implements IgniteSpiTimeoutObject {
         /** */
-        private final long id = idGen.incrementAndGet();
+        private final IgniteUuid id = IgniteUuid.randomUuid();
 
         /** */
         private final T obj;
@@ -2533,34 +2402,24 @@ public class TcpCommunicationSpi extends IgniteSpiAdapter
             return done.compareAndSet(false, true);
         }
 
-        /**
-         * @return {@code True} if object has not yet been canceled.
-         */
-        boolean onTimeout() {
+        /** {@inheritDoc} */
+        @Override public void onTimeout() {
             if (done.compareAndSet(false, true)) {
                 // Close socket - timeout occurred.
                 if (obj instanceof GridCommunicationClient)
                     ((GridCommunicationClient)obj).forceClose();
                 else
                     U.closeQuiet((AbstractInterruptibleChannel)obj);
-
-                return true;
             }
-
-            return false;
         }
 
-        /**
-         * @return End time.
-         */
-        long endTime() {
+        /** {@inheritDoc} */
+        @Override public long endTime() {
             return endTime;
         }
 
-        /**
-         * @return ID.
-         */
-        long id() {
+        /** {@inheritDoc} */
+        @Override public IgniteUuid id() {
             return id;
         }
 

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/873e01bb/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/ClientImpl.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/ClientImpl.java b/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/ClientImpl.java
index e672d64..5b66019 100644
--- a/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/ClientImpl.java
+++ b/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/ClientImpl.java
@@ -116,7 +116,6 @@ class ClientImpl extends TcpDiscoveryImpl {
         b.append("    Message worker: ").append(threadStatus(msgWorker)).append(U.nl());
         b.append("    Socket reader: ").append(threadStatus(sockReader)).append(U.nl());
         b.append("    Socket writer: ").append(threadStatus(sockWriter)).append(U.nl());
-        b.append("    Socket timeout worker: ").append(threadStatus(spi.sockTimeoutWorker)).append(U.nl());
 
         b.append(U.nl());
 
@@ -524,11 +523,9 @@ class ClientImpl extends TcpDiscoveryImpl {
 
         U.interrupt(sockWriter);
         U.interrupt(msgWorker);
-        U.interrupt(spi.sockTimeoutWorker);
 
         U.join(sockWriter, log);
         U.join(msgWorker, log);
-        U.join(spi.sockTimeoutWorker, log);
     }
 
     /** {@inheritDoc} */

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/873e01bb/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/ServerImpl.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/ServerImpl.java b/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/ServerImpl.java
index 57c13d6..485f57d 100644
--- a/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/ServerImpl.java
+++ b/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/ServerImpl.java
@@ -1412,7 +1412,6 @@ class ServerImpl extends TcpDiscoveryImpl {
             b.append("    Message worker: ").append(threadStatus(msgWorker)).append(U.nl());
             b.append("    Check status sender: ").append(threadStatus(chkStatusSnd)).append(U.nl());
             b.append("    HB sender: ").append(threadStatus(hbsSnd)).append(U.nl());
-            b.append("    Socket timeout worker: ").append(threadStatus(spi.sockTimeoutWorker)).append(U.nl());
             b.append("    IP finder cleaner: ").append(threadStatus(ipFinderCleaner)).append(U.nl());
             b.append("    Stats printer: ").append(threadStatus(statsPrinter)).append(U.nl());
 

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/873e01bb/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/TcpDiscoverySpi.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/TcpDiscoverySpi.java b/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/TcpDiscoverySpi.java
index 56fb63f..8365716 100644
--- a/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/TcpDiscoverySpi.java
+++ b/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/TcpDiscoverySpi.java
@@ -255,9 +255,6 @@ public class TcpDiscoverySpi extends IgniteSpiAdapter implements DiscoverySpi, T
     /** Internal and external addresses of local node. */
     protected Collection<InetSocketAddress> locNodeAddrs;
 
-    /** Socket timeout worker. */
-    protected SocketTimeoutWorker sockTimeoutWorker;
-
     /** Start time of the very first grid node. */
     protected volatile long gridStartTime;
 
@@ -1118,7 +1115,7 @@ public class TcpDiscoverySpi extends IgniteSpiAdapter implements DiscoverySpi, T
 
         SocketTimeoutObject obj = new SocketTimeoutObject(sock, U.currentTimeMillis() + sockTimeout);
 
-        sockTimeoutWorker.addTimeoutObject(obj);
+        addTimeoutObject(obj);
 
         IOException err = null;
 
@@ -1136,7 +1133,7 @@ public class TcpDiscoverySpi extends IgniteSpiAdapter implements DiscoverySpi, T
             boolean cancelled = obj.cancel();
 
             if (cancelled)
-                sockTimeoutWorker.removeTimeoutObject(obj);
+                removeTimeoutObject(obj);
 
             // Throw original exception.
             if (err != null)
@@ -1180,7 +1177,7 @@ public class TcpDiscoverySpi extends IgniteSpiAdapter implements DiscoverySpi, T
 
         SocketTimeoutObject obj = new SocketTimeoutObject(sock, U.currentTimeMillis() + sockTimeout);
 
-        sockTimeoutWorker.addTimeoutObject(obj);
+        addTimeoutObject(obj);
 
         IOException err = null;
 
@@ -1198,7 +1195,7 @@ public class TcpDiscoverySpi extends IgniteSpiAdapter implements DiscoverySpi, T
             boolean cancelled = obj.cancel();
 
             if (cancelled)
-                sockTimeoutWorker.removeTimeoutObject(obj);
+                removeTimeoutObject(obj);
 
             // Throw original exception.
             if (err != null)
@@ -1222,7 +1219,7 @@ public class TcpDiscoverySpi extends IgniteSpiAdapter implements DiscoverySpi, T
 
         SocketTimeoutObject obj = new SocketTimeoutObject(sock, U.currentTimeMillis() + sockTimeout);
 
-        sockTimeoutWorker.addTimeoutObject(obj);
+        addTimeoutObject(obj);
 
         OutputStream out = sock.getOutputStream();
 
@@ -1240,7 +1237,7 @@ public class TcpDiscoverySpi extends IgniteSpiAdapter implements DiscoverySpi, T
             boolean cancelled = obj.cancel();
 
             if (cancelled)
-                sockTimeoutWorker.removeTimeoutObject(obj);
+                removeTimeoutObject(obj);
 
             // Throw original exception.
             if (err != null)
@@ -1599,9 +1596,6 @@ public class TcpDiscoverySpi extends IgniteSpiAdapter implements DiscoverySpi, T
                 mcastIpFinder.setLocalAddress(locAddr);
         }
 
-        sockTimeoutWorker = new SocketTimeoutWorker();
-        sockTimeoutWorker.start();
-
         impl.spiStart(gridName);
     }
 
@@ -1611,9 +1605,6 @@ public class TcpDiscoverySpi extends IgniteSpiAdapter implements DiscoverySpi, T
             // Safety.
             ctxInitLatch.countDown();
 
-        U.interrupt(sockTimeoutWorker);
-        U.join(sockTimeoutWorker, log);
-
         if (ipFinder != null) {
             try {
                 ipFinder.close();
@@ -1754,117 +1745,11 @@ public class TcpDiscoverySpi extends IgniteSpiAdapter implements DiscoverySpi, T
     }
 
     /**
-     * Handles sockets timeouts.
-     */
-    protected class SocketTimeoutWorker extends IgniteSpiThread {
-        /** Time-based sorted set for timeout objects. */
-        private final GridConcurrentSkipListSet<SocketTimeoutObject> timeoutObjs =
-            new GridConcurrentSkipListSet<>(new Comparator<SocketTimeoutObject>() {
-                @Override public int compare(SocketTimeoutObject o1, SocketTimeoutObject o2) {
-                    int res = Long.compare(o1.endTime(), o2.endTime());
-
-                    if (res != 0)
-                        return res;
-
-                    return Long.compare(o1.id(), o2.id());
-                }
-            });
-
-        /** Mutex. */
-        private final Object mux0 = new Object();
-
-        /**
-         *
-         */
-        SocketTimeoutWorker() {
-            super(gridName, "tcp-disco-sock-timeout-worker", log);
-
-            setPriority(threadPri);
-        }
-
-        /**
-         * @param timeoutObj Timeout object to add.
-         */
-        @SuppressWarnings({"NakedNotify"})
-        public void addTimeoutObject(SocketTimeoutObject timeoutObj) {
-            assert timeoutObj != null && timeoutObj.endTime() > 0 && timeoutObj.endTime() != Long.MAX_VALUE;
-
-            timeoutObjs.add(timeoutObj);
-
-            if (timeoutObjs.firstx() == timeoutObj) {
-                synchronized (mux0) {
-                    mux0.notifyAll();
-                }
-            }
-        }
-
-        /**
-         * @param timeoutObj Timeout object to remove.
-         */
-        public void removeTimeoutObject(SocketTimeoutObject timeoutObj) {
-            assert timeoutObj != null;
-
-            timeoutObjs.remove(timeoutObj);
-        }
-
-        /** {@inheritDoc} */
-        @Override protected void body() throws InterruptedException {
-            if (log.isDebugEnabled())
-                log.debug("Socket timeout worker has been started.");
-
-            while (!isInterrupted()) {
-                long now = U.currentTimeMillis();
-
-                for (Iterator<SocketTimeoutObject> iter = timeoutObjs.iterator(); iter.hasNext(); ) {
-                    SocketTimeoutObject timeoutObj = iter.next();
-
-                    if (timeoutObj.endTime() <= now) {
-                        iter.remove();
-
-                        if (timeoutObj.onTimeout()) {
-                            LT.warn(log, null, "Socket write has timed out (consider increasing " +
-                                "'sockTimeout' configuration property) [sockTimeout=" + sockTimeout + ']');
-
-                            stats.onSocketTimeout();
-                        }
-                    }
-                    else
-                        break;
-                }
-
-                synchronized (mux0) {
-                    while (true) {
-                        // Access of the first element must be inside of
-                        // synchronization block, so we don't miss out
-                        // on thread notification events sent from
-                        // 'addTimeoutObject(..)' method.
-                        SocketTimeoutObject first = timeoutObjs.firstx();
-
-                        if (first != null) {
-                            long waitTime = first.endTime() - U.currentTimeMillis();
-
-                            if (waitTime > 0)
-                                mux0.wait(waitTime);
-                            else
-                                break;
-                        }
-                        else
-                            mux0.wait(5000);
-                    }
-                }
-            }
-        }
-    }
-
-    /**
      * Socket timeout object.
      */
-    private static class SocketTimeoutObject {
+    private class SocketTimeoutObject implements IgniteSpiTimeoutObject {
         /** */
-        private static final AtomicLong idGen = new AtomicLong();
-
-        /** */
-        private final long id = idGen.incrementAndGet();
+        private final IgniteUuid id = IgniteUuid.randomUuid();
 
         /** */
         private final Socket sock;
@@ -1894,31 +1779,26 @@ public class TcpDiscoverySpi extends IgniteSpiAdapter implements DiscoverySpi, T
             return done.compareAndSet(false, true);
         }
 
-        /**
-         * @return {@code True} if object has not yet been canceled.
-         */
-        boolean onTimeout() {
+        /** {@inheritDoc} */
+        @Override public void onTimeout() {
             if (done.compareAndSet(false, true)) {
                 // Close socket - timeout occurred.
                 U.closeQuiet(sock);
 
-                return true;
-            }
+                LT.warn(log, null, "Socket write has timed out (consider increasing " +
+                    "'sockTimeout' configuration property) [sockTimeout=" + sockTimeout + ']');
 
-            return false;
+                stats.onSocketTimeout();
+            }
         }
 
-        /**
-         * @return End time.
-         */
-        long endTime() {
+        /** {@inheritDoc} */
+        @Override public long endTime() {
             return endTime;
         }
 
-        /**
-         * @return ID.
-         */
-        long id() {
+        /** {@inheritDoc} */
+        @Override public  IgniteUuid id() {
             return id;
         }
 

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/873e01bb/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/datastructures/IgniteCountDownLatchAbstractSelfTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/datastructures/IgniteCountDownLatchAbstractSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/datastructures/IgniteCountDownLatchAbstractSelfTest.java
index 0f2a898..80e6123 100644
--- a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/datastructures/IgniteCountDownLatchAbstractSelfTest.java
+++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/datastructures/IgniteCountDownLatchAbstractSelfTest.java
@@ -28,6 +28,7 @@ import org.jetbrains.annotations.*;
 import java.io.*;
 import java.util.*;
 import java.util.concurrent.*;
+import java.util.concurrent.atomic.*;
 
 import static java.util.concurrent.TimeUnit.*;
 
@@ -258,6 +259,107 @@ public abstract class IgniteCountDownLatchAbstractSelfTest extends IgniteAtomics
         checkRemovedLatch(latch);
     }
 
+    /**
+     * @throws Exception If failed.
+     */
+    public void testLatchMultinode1() throws Exception {
+        if (gridCount() == 1)
+            return;
+
+        IgniteCountDownLatch latch = grid(0).countDownLatch("l1", 10,
+            true,
+            true);
+
+        List<IgniteInternalFuture<?>> futs = new ArrayList<>();
+
+        final AtomicBoolean countedDown = new AtomicBoolean();
+
+        for (int i = 0; i < gridCount(); i++) {
+            final Ignite ignite = grid(i);
+
+            futs.add(GridTestUtils.runAsync(new Callable<Void>() {
+                @Override public Void call() throws Exception {
+                    IgniteCountDownLatch latch = ignite.countDownLatch("l1", 10,
+                        true,
+                        false);
+
+                    assertNotNull(latch);
+
+                    boolean wait = latch.await(30_000);
+
+                    assertTrue(countedDown.get());
+
+                    assertEquals(0, latch.count());
+
+                    assertTrue(wait);
+
+                    return null;
+                }
+            }));
+        }
+
+        for (int i = 0; i < 10; i++) {
+            if (i == 9)
+                countedDown.set(true);
+
+            latch.countDown();
+        }
+
+        for (IgniteInternalFuture<?> fut : futs)
+            fut.get(30_000);
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testLatchMultinode2() throws Exception {
+        if (gridCount() == 1)
+            return;
+
+        IgniteCountDownLatch latch = grid(0).countDownLatch("l2", gridCount() * 3,
+            true,
+            true);
+
+        assertNotNull(latch);
+
+        List<IgniteInternalFuture<?>> futs = new ArrayList<>();
+
+        final AtomicInteger cnt = new AtomicInteger();
+
+        for (int i = 0; i < gridCount(); i++) {
+            final Ignite ignite = grid(i);
+
+            futs.add(GridTestUtils.runAsync(new Callable<Void>() {
+                @Override public Void call() throws Exception {
+                    IgniteCountDownLatch latch = ignite.countDownLatch("l2", 10,
+                        true,
+                        false);
+
+                    assertNotNull(latch);
+
+                    for (int i = 0; i < 3; i++) {
+                        cnt.incrementAndGet();
+
+                        latch.countDown();
+                    }
+
+                    boolean wait = latch.await(30_000);
+
+                    assertEquals(gridCount() * 3, cnt.get());
+
+                    assertEquals(0, latch.count());
+
+                    assertTrue(wait);
+
+                    return null;
+                }
+            }));
+        }
+
+        for (IgniteInternalFuture<?> fut : futs)
+            fut.get(30_000);
+    }
+
     /** {@inheritDoc} */
     @Override public void writeExternal(ObjectOutput out) throws IOException {
         // No-op.

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/873e01bb/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/expiry/IgniteCacheClientNearCacheExpiryTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/expiry/IgniteCacheClientNearCacheExpiryTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/expiry/IgniteCacheClientNearCacheExpiryTest.java
new file mode 100644
index 0000000..602ac18
--- /dev/null
+++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/expiry/IgniteCacheClientNearCacheExpiryTest.java
@@ -0,0 +1,103 @@
+/*
+ * 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.expiry;
+
+import org.apache.ignite.*;
+import org.apache.ignite.cache.*;
+import org.apache.ignite.configuration.*;
+import org.apache.ignite.internal.processors.cache.*;
+import org.apache.ignite.internal.util.typedef.internal.*;
+
+import javax.cache.expiry.*;
+
+import java.util.concurrent.*;
+
+import static org.apache.ignite.cache.CacheAtomicityMode.*;
+import static org.apache.ignite.cache.CacheMode.*;
+
+/**
+ *
+ */
+public class IgniteCacheClientNearCacheExpiryTest extends IgniteCacheAbstractTest {
+    /** */
+    private static final int NODES = 3;
+
+    /** {@inheritDoc} */
+    @Override protected int gridCount() {
+        return NODES;
+    }
+
+    /** {@inheritDoc} */
+    @Override protected CacheMode cacheMode() {
+        return PARTITIONED;
+    }
+
+    /** {@inheritDoc} */
+    @Override protected CacheAtomicityMode atomicityMode() {
+        return TRANSACTIONAL;
+    }
+
+    /** {@inheritDoc} */
+    @Override protected NearCacheConfiguration nearConfiguration() {
+        return new NearCacheConfiguration();
+    }
+
+    /** {@inheritDoc} */
+    @Override protected IgniteConfiguration getConfiguration(String gridName) throws Exception {
+        IgniteConfiguration cfg = super.getConfiguration(gridName);
+
+        if (gridName.equals(getTestGridName(NODES - 1)))
+            cfg.setClientMode(true);
+
+        return cfg;
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testExpirationOnClient() throws Exception {
+        Ignite ignite = grid(NODES - 1);
+
+        assertTrue(ignite.configuration().isClientMode());
+
+        IgniteCache<Object, Object> cache = ignite.cache(null);
+
+        assertTrue(((IgniteCacheProxy)cache).context().isNear());
+
+        for (int i = 0 ; i < 100; i++)
+            cache.put(i, i);
+
+        CreatedExpiryPolicy plc = new CreatedExpiryPolicy(new Duration(TimeUnit.MILLISECONDS, 500));
+
+        IgniteCache<Object, Object> cacheWithExpiry = cache.withExpiryPolicy(plc);
+
+        for (int i = 100 ; i < 200; i++) {
+            cacheWithExpiry.put(i, i);
+
+            assertEquals(i, cacheWithExpiry.localPeek(i));
+        }
+
+        U.sleep(1000);
+
+        for (int i = 0 ; i < 100; i++)
+            assertEquals(i, cacheWithExpiry.localPeek(i));
+
+        for (int i = 100 ; i < 200; i++)
+            assertNull(cache.localPeek(i));
+    }
+}

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/873e01bb/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/expiry/IgniteCacheExpiryPolicyTestSuite.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/expiry/IgniteCacheExpiryPolicyTestSuite.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/expiry/IgniteCacheExpiryPolicyTestSuite.java
index c006f69..c78ec5c 100644
--- a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/expiry/IgniteCacheExpiryPolicyTestSuite.java
+++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/expiry/IgniteCacheExpiryPolicyTestSuite.java
@@ -50,6 +50,8 @@ public class IgniteCacheExpiryPolicyTestSuite extends TestSuite {
 
         suite.addTestSuite(IgniteCacheTtlCleanupSelfTest.class);
 
+        suite.addTestSuite(IgniteCacheClientNearCacheExpiryTest.class);
+
         return suite;
     }
 }

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/873e01bb/modules/core/src/test/java/org/apache/ignite/testframework/GridSpiTestContext.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/testframework/GridSpiTestContext.java b/modules/core/src/test/java/org/apache/ignite/testframework/GridSpiTestContext.java
index 5867fb8..21f9424 100644
--- a/modules/core/src/test/java/org/apache/ignite/testframework/GridSpiTestContext.java
+++ b/modules/core/src/test/java/org/apache/ignite/testframework/GridSpiTestContext.java
@@ -501,6 +501,16 @@ public class GridSpiTestContext implements IgniteSpiContext {
         return false;
     }
 
+    /** {@inheritDoc} */
+    @Override public void addTimeoutObject(IgniteSpiTimeoutObject obj) {
+        // No-op.
+    }
+
+    /** {@inheritDoc} */
+    @Override public void removeTimeoutObject(IgniteSpiTimeoutObject obj) {
+        // No-op.
+    }
+
     /**
      * @param cacheName Cache name.
      * @return Map representing cache.


[11/42] incubator-ignite git commit: # ignite-883

Posted by ag...@apache.org.
# ignite-883


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

Branch: refs/heads/ignite-389
Commit: 6bfc78ea6a4559b4ee9059893e0a7f9d4195ad3c
Parents: a3eb572
Author: sboikov <sb...@gridgain.com>
Authored: Thu Jun 4 10:04:18 2015 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Thu Jun 4 10:04:18 2015 +0300

----------------------------------------------------------------------
 .../apache/ignite/internal/IgniteKernal.java    | 18 ++++++++--
 .../datastructures/DataStructuresProcessor.java | 36 +++++++++++++++++---
 2 files changed, 48 insertions(+), 6 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/6bfc78ea/modules/core/src/main/java/org/apache/ignite/internal/IgniteKernal.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/IgniteKernal.java b/modules/core/src/main/java/org/apache/ignite/internal/IgniteKernal.java
index e3fc50f..27d2dc7 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/IgniteKernal.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/IgniteKernal.java
@@ -167,6 +167,14 @@ public class IgniteKernal implements IgniteEx, IgniteMXBean, Externalizable {
     @GridToStringExclude
     private Timer updateNtfTimer;
 
+    /** */
+    @GridToStringExclude
+    private GridTimeoutProcessor.CancelableTask starveTask;
+
+    /** */
+    @GridToStringExclude
+    private GridTimeoutProcessor.CancelableTask metricsLogTask;
+
     /** Indicate error on grid stop. */
     @GridToStringExclude
     private boolean errOnStop;
@@ -859,7 +867,7 @@ public class IgniteKernal implements IgniteEx, IgniteMXBean, Externalizable {
         if (starveCheck) {
             final long interval = F.isEmpty(intervalStr) ? PERIODIC_STARVATION_CHECK_FREQ : Long.parseLong(intervalStr);
 
-            ctx.timeout().schedule(new Runnable() {
+            starveTask = ctx.timeout().schedule(new Runnable() {
                 /** Last completed task count. */
                 private long lastCompletedCnt;
 
@@ -886,7 +894,7 @@ public class IgniteKernal implements IgniteEx, IgniteMXBean, Externalizable {
         long metricsLogFreq = cfg.getMetricsLogFrequency();
 
         if (metricsLogFreq > 0) {
-            ctx.timeout().schedule(new Runnable() {
+            metricsLogTask = ctx.timeout().schedule(new Runnable() {
                 private final DecimalFormat dblFmt = new DecimalFormat("#.##");
 
                 @Override public void run() {
@@ -1700,6 +1708,12 @@ public class IgniteKernal implements IgniteEx, IgniteMXBean, Externalizable {
             if (updateNtfTimer != null)
                 updateNtfTimer.cancel();
 
+            if (starveTask != null)
+                starveTask.close();
+
+            if (metricsLogTask != null)
+                metricsLogTask.close();
+
             boolean interrupted = false;
 
             while (true) {

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/6bfc78ea/modules/core/src/main/java/org/apache/ignite/internal/processors/datastructures/DataStructuresProcessor.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/datastructures/DataStructuresProcessor.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/datastructures/DataStructuresProcessor.java
index 27f6a29..2138639 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/datastructures/DataStructuresProcessor.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/datastructures/DataStructuresProcessor.java
@@ -18,6 +18,7 @@
 package org.apache.ignite.internal.processors.datastructures;
 
 import org.apache.ignite.*;
+import org.apache.ignite.cache.*;
 import org.apache.ignite.configuration.*;
 import org.apache.ignite.internal.*;
 import org.apache.ignite.internal.cluster.*;
@@ -145,7 +146,7 @@ public final class DataStructuresProcessor extends GridProcessorAdapter {
             dsCacheCtx = atomicsCache.context();
 
             qryId = dsCacheCtx.continuousQueries().executeInternalQuery(new DataStructuresEntryListener(),
-                null,
+                new DataStructuresEntryFilter(),
                 dsCacheCtx.isReplicated() && dsCacheCtx.affinityNode(),
                 false);
         }
@@ -1051,10 +1052,37 @@ public final class DataStructuresProcessor extends GridProcessorAdapter {
     /**
      *
      */
-    private class DataStructuresEntryListener implements CacheEntryUpdatedListener<GridCacheInternalKey, GridCacheInternal> {
+    static class DataStructuresEntryFilter implements CacheEntryEventSerializableFilter<Object, Object> {
+        /** */
+        private static final long serialVersionUID = 0L;
+
         /** {@inheritDoc} */
-        @Override public void onUpdated(Iterable<CacheEntryEvent<? extends GridCacheInternalKey, ? extends GridCacheInternal>> evts)
-            throws CacheEntryListenerException {
+        @Override public boolean evaluate(CacheEntryEvent<?, ?> evt) throws CacheEntryListenerException {
+            if (evt.getEventType() == EventType.CREATED || evt.getEventType() == EventType.UPDATED)
+                return evt.getValue() instanceof GridCacheCountDownLatchValue;
+            else {
+                assert evt.getEventType() == EventType.REMOVED : evt;
+
+                return true;
+            }
+        }
+
+        /** {@inheritDoc} */
+        @Override public String toString() {
+            return S.toString(DataStructuresEntryFilter.class, this);
+        }
+    }
+
+    /**
+     *
+     */
+    private class DataStructuresEntryListener implements
+        CacheEntryUpdatedListener<GridCacheInternalKey, GridCacheInternal> {
+        /** {@inheritDoc} */
+        @Override public void onUpdated(
+            Iterable<CacheEntryEvent<? extends GridCacheInternalKey, ? extends GridCacheInternal>> evts)
+            throws CacheEntryListenerException
+        {
             for (CacheEntryEvent<? extends GridCacheInternalKey, ? extends GridCacheInternal> evt : evts) {
                 if (evt.getEventType() == EventType.CREATED || evt.getEventType() == EventType.UPDATED) {
                     GridCacheInternal val0 = evt.getValue();


[03/42] incubator-ignite git commit: # IGNITE-883 Don't create public pool on client node

Posted by ag...@apache.org.
# IGNITE-883 Don't create public pool on client node


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

Branch: refs/heads/ignite-389
Commit: bfae889766d92a126a5fc2938b196f83249cc097
Parents: dce6789
Author: sevdokimov <se...@gridgain.com>
Authored: Mon Jun 1 18:10:28 2015 +0300
Committer: sevdokimov <se...@gridgain.com>
Committed: Mon Jun 1 19:49:54 2015 +0300

----------------------------------------------------------------------
 .../org/apache/ignite/internal/IgnitionEx.java  | 36 +++++++++++++-------
 1 file changed, 24 insertions(+), 12 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/bfae8897/modules/core/src/main/java/org/apache/ignite/internal/IgnitionEx.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/IgnitionEx.java b/modules/core/src/main/java/org/apache/ignite/internal/IgnitionEx.java
index 6e4efb5..d9f6bb4 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/IgnitionEx.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/IgnitionEx.java
@@ -1447,16 +1447,6 @@ public class IgnitionEx {
                 ensureMultiInstanceSupport(myCfg.getSwapSpaceSpi());
             }
 
-            execSvc = new IgniteThreadPoolExecutor(
-                "pub-" + cfg.getGridName(),
-                cfg.getPublicThreadPoolSize(),
-                cfg.getPublicThreadPoolSize(),
-                DFLT_PUBLIC_KEEP_ALIVE_TIME,
-                new LinkedBlockingQueue<Runnable>(DFLT_PUBLIC_THREADPOOL_QUEUE_CAP));
-
-            // Pre-start all threads as they are guaranteed to be needed.
-            ((ThreadPoolExecutor) execSvc).prestartAllCoreThreads();
-
             // Note that since we use 'LinkedBlockingQueue', number of
             // maximum threads has no effect.
             sysExecSvc = new IgniteThreadPoolExecutor(
@@ -1466,8 +1456,30 @@ public class IgnitionEx {
                 DFLT_SYSTEM_KEEP_ALIVE_TIME,
                 new LinkedBlockingQueue<Runnable>(DFLT_SYSTEM_THREADPOOL_QUEUE_CAP));
 
-            // Pre-start all threads as they are guaranteed to be needed.
-            ((ThreadPoolExecutor) sysExecSvc).prestartAllCoreThreads();
+            boolean isClientMode = Boolean.TRUE.equals(myCfg.isClientMode());
+
+            if (isClientMode) {
+                execSvc = new IgniteThreadPoolExecutor(
+                    "pub-" + cfg.getGridName(),
+                    0,
+                    cfg.getPublicThreadPoolSize(),
+                    2000,
+                    new LinkedBlockingQueue<Runnable>(DFLT_PUBLIC_THREADPOOL_QUEUE_CAP));
+            }
+            else {
+                execSvc = new IgniteThreadPoolExecutor(
+                    "pub-" + cfg.getGridName(),
+                    cfg.getPublicThreadPoolSize(),
+                    cfg.getPublicThreadPoolSize(),
+                    DFLT_PUBLIC_KEEP_ALIVE_TIME,
+                    new LinkedBlockingQueue<Runnable>(DFLT_PUBLIC_THREADPOOL_QUEUE_CAP));
+
+                // Pre-start all threads as they are guaranteed to be needed.
+                ((ThreadPoolExecutor) execSvc).prestartAllCoreThreads();
+
+                // Pre-start all threads as they are guaranteed to be needed.
+                ((ThreadPoolExecutor) sysExecSvc).prestartAllCoreThreads();
+            }
 
             // Note that since we use 'LinkedBlockingQueue', number of
             // maximum threads has no effect.


[35/42] incubator-ignite git commit: # ignite-sprint-5 sanity check for classname.properies load

Posted by ag...@apache.org.
# ignite-sprint-5 sanity check for classname.properies load


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

Branch: refs/heads/ignite-389
Commit: 8467a3c36414192268b8d852715fc501f53b4772
Parents: a5b5ec7
Author: sboikov <sb...@gridgain.com>
Authored: Mon Jun 8 12:42:36 2015 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Mon Jun 8 12:42:36 2015 +0300

----------------------------------------------------------------------
 .../internal/MarshallerContextAdapter.java      | 36 ++++++++++++++++++--
 1 file changed, 34 insertions(+), 2 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/8467a3c3/modules/core/src/main/java/org/apache/ignite/internal/MarshallerContextAdapter.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/MarshallerContextAdapter.java b/modules/core/src/main/java/org/apache/ignite/internal/MarshallerContextAdapter.java
index 5dca2f2..21f2264 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/MarshallerContextAdapter.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/MarshallerContextAdapter.java
@@ -18,6 +18,7 @@
 package org.apache.ignite.internal;
 
 import org.apache.ignite.*;
+import org.apache.ignite.internal.processors.cache.distributed.dht.preloader.*;
 import org.apache.ignite.internal.util.typedef.internal.*;
 import org.apache.ignite.marshaller.*;
 import org.jsr166.*;
@@ -49,10 +50,29 @@ public abstract class MarshallerContextAdapter implements MarshallerContext {
 
             Enumeration<URL> urls = ldr.getResources(CLS_NAMES_FILE);
 
-            while (urls.hasMoreElements())
+            boolean foundClsNames = false;
+
+            while (urls.hasMoreElements()) {
                 processResource(urls.nextElement());
 
-            processResource(ldr.getResource(JDK_CLS_NAMES_FILE));
+                foundClsNames = true;
+            }
+
+            if (!foundClsNames)
+                throw new IgniteException("Failed to load class names properties file packaged with ignite binaries " +
+                    "[file=" + CLS_NAMES_FILE + ", ldr=" + ldr + ']');
+
+            URL jdkClsNames = ldr.getResource(JDK_CLS_NAMES_FILE);
+
+            if (jdkClsNames == null)
+                throw new IgniteException("Failed to load class names properties file packaged with ignite binaries " +
+                    "[file=" + JDK_CLS_NAMES_FILE + ", ldr=" + ldr + ']');
+
+            processResource(jdkClsNames);
+
+            checkHasClassName(GridDhtPartitionFullMap.class.getName(), ldr, CLS_NAMES_FILE);
+            checkHasClassName(GridDhtPartitionMap.class.getName(), ldr, CLS_NAMES_FILE);
+            checkHasClassName(HashMap.class.getName(), ldr, JDK_CLS_NAMES_FILE);
         }
         catch (IOException e) {
             throw new IllegalStateException("Failed to initialize marshaller context.", e);
@@ -60,6 +80,18 @@ public abstract class MarshallerContextAdapter implements MarshallerContext {
     }
 
     /**
+     * @param clsName Class name.
+     * @param ldr Class loader used to get properties file.
+     * @param fileName File name.
+     */
+    private void checkHasClassName(String clsName, ClassLoader ldr, String fileName) {
+        if (!map.containsKey(clsName.hashCode()))
+            throw new IgniteException("Failed to read class name from class names properties file. " +
+                "Make sure class names properties file packaged with ignite binaries is not corrupted " +
+                "[clsName=" + clsName + ", fileName=" + fileName + ", ldr=" + ldr + ']');
+    }
+
+    /**
      * @param url Resource URL.
      * @throws IOException In case of error.
      */


[28/42] incubator-ignite git commit: Merge branches 'ignite-941' and 'ignite-sprint-5' of https://git-wip-us.apache.org/repos/asf/incubator-ignite into ignite-941

Posted by ag...@apache.org.
Merge branches 'ignite-941' and 'ignite-sprint-5' of https://git-wip-us.apache.org/repos/asf/incubator-ignite into ignite-941


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

Branch: refs/heads/ignite-389
Commit: 59db4a5e0f1dd83d56cb95f0a4184451f0a387fd
Parents: b4d1915 ab20fd9
Author: Valentin Kulichenko <vk...@gridgain.com>
Authored: Fri Jun 5 13:55:21 2015 -0700
Committer: Valentin Kulichenko <vk...@gridgain.com>
Committed: Fri Jun 5 13:55:21 2015 -0700

----------------------------------------------------------------------
 .../java/org/apache/ignite/IgniteCache.java     |  11 ++
 .../affinity/GridAffinityProcessor.java         |  23 ++++-
 .../processors/cache/GridCacheAdapter.java      |  17 ++-
 .../processors/cache/GridCacheProxyImpl.java    |  14 ++-
 .../processors/cache/IgniteCacheProxy.java      |  23 +++++
 .../processors/cache/IgniteInternalCache.java   |  11 +-
 .../datastreamer/DataStreamerImpl.java          |  92 ++++++++++++-----
 .../processors/igfs/IgfsMetaManager.java        |   2 +-
 .../ignite/internal/visor/cache/VisorCache.java |   2 +-
 .../VisorCacheConfigurationCollectorJob.java    |   6 +-
 .../internal/visor/cache/VisorCacheMetrics.java |  19 ++--
 .../cache/VisorCacheMetricsCollectorTask.java   |  10 +-
 .../cache/VisorCacheStoreConfiguration.java     |   5 +-
 .../cache/GridCacheAbstractFullApiSelfTest.java |  83 +++++++++++++++
 .../GridCacheExAbstractFullApiSelfTest.java     | 103 -------------------
 .../GridCacheExColocatedFullApiSelfTest.java    |  33 ------
 .../near/GridCacheExNearFullApiSelfTest.java    |  39 -------
 .../GridCacheExReplicatedFullApiSelfTest.java   |  33 ------
 .../local/GridCacheExLocalFullApiSelfTest.java  |  30 ------
 .../DataStreamerMultiThreadedSelfTest.java      |  59 ++++++++---
 .../IgniteMessagingWithClientTest.java          |   2 +
 .../IgniteCacheFullApiSelfTestSuite.java        |   6 --
 22 files changed, 309 insertions(+), 314 deletions(-)
----------------------------------------------------------------------



[10/42] incubator-ignite git commit: Merge remote-tracking branch 'remotes/origin/ignite-sprint-5' into ignite-883_1

Posted by ag...@apache.org.
Merge remote-tracking branch 'remotes/origin/ignite-sprint-5' into ignite-883_1


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

Branch: refs/heads/ignite-389
Commit: a3eb572ceeeb09f293f613ec6a572f8bc2a77ca0
Parents: 4184f36 ae5189a
Author: sboikov <sb...@gridgain.com>
Authored: Thu Jun 4 09:50:46 2015 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Thu Jun 4 09:50:46 2015 +0300

----------------------------------------------------------------------
 .../processors/cache/GridCacheContext.java      |   3 -
 .../dht/GridDhtPartitionTopologyImpl.java       |   8 +-
 .../GridDhtPartitionsExchangeFuture.java        |  10 +-
 .../ignite/spi/discovery/tcp/ClientImpl.java    |   3 +
 .../ignite/spi/discovery/tcp/ServerImpl.java    |  31 -----
 .../spi/discovery/tcp/TcpDiscoveryImpl.java     |  56 +++++++-
 .../tcp/ipfinder/TcpDiscoveryIpFinder.java      |  10 +-
 .../TcpDiscoveryMulticastIpFinder.java          |  47 +++++--
 .../cache/IgniteDynamicCacheStartSelfTest.java  |  62 +++++++++
 .../tcp/TcpClientDiscoverySpiMulticastTest.java | 129 +++++++++++++++++++
 .../IgniteSpiDiscoverySelfTestSuite.java        |   1 +
 .../gce/TcpDiscoveryGoogleStorageIpFinder.java  |  43 ++++---
 12 files changed, 321 insertions(+), 82 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/a3eb572c/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/ClientImpl.java
----------------------------------------------------------------------

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/a3eb572c/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/ServerImpl.java
----------------------------------------------------------------------


[39/42] incubator-ignite git commit: Devnotes sprint-5

Posted by ag...@apache.org.
Devnotes sprint-5


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

Branch: refs/heads/ignite-389
Commit: f1cfd2985803c8b05462737b847f956ea043341a
Parents: 015afdb
Author: avinogradov <av...@gridgain.com>
Authored: Mon Jun 8 15:17:54 2015 +0300
Committer: avinogradov <av...@gridgain.com>
Committed: Mon Jun 8 15:17:54 2015 +0300

----------------------------------------------------------------------
 DEVNOTES.txt | 42 +++++++++++++++++++++++++++++-------------
 1 file changed, 29 insertions(+), 13 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/f1cfd298/DEVNOTES.txt
----------------------------------------------------------------------
diff --git a/DEVNOTES.txt b/DEVNOTES.txt
index 1562dc4..b7fea83 100644
--- a/DEVNOTES.txt
+++ b/DEVNOTES.txt
@@ -24,12 +24,14 @@ NOTE: JDK version should be 1.7.0-* or >= 1.8.0-u40.
 
 Ignite Release Instructions
 ===========================
-Use people.apache.org/keys/committer/<username>.asc key to generate KEYS file.
-Execute these commands at source root:
+Use your people.apache.org/keys/committer/<username>.asc key to generate KEYS file.
+Download https://dist.apache.org/repos/dist/release/incubator/ignite/KEYS and append you key using commands:
 
 gpg --list-sigs <keyname> >> KEYS
 gpg --armor --export <keyname> >> KEYS
 
+Upload modified KEYS file.
+
 Specify gpg profile at settings.xml. It will be used to sign sources and artifacts.
 
 <profile>
@@ -43,24 +45,38 @@ Specify gpg profile at settings.xml. It will be used to sign sources and artifac
 Ensure you have RELEASE (not SNAPSHOT) version at Ignite poms.
 Maven release plugin release:prepare goal can be used to make release tag.
 
-Deploy Ignite release candidate to maven repository and dev-svn:
+Deploy Ignite release candidate to maven repository and dev-svn, make tag:
+
+   Following command deploys Ignite to maven repository, prepares sources and fabric edition binaries.
+      mvn deploy -P apache-release,gpg,release,scala,lgpl,deploy-ignite-site -Dignite.edition=fabric -DskipTests -B
+
+   In case you want to release both fabric and hadoop editions you have to build hadoop edition first using command
+      mvn package -P apache-release,gpg,release,scala,lgpl -Dignite.edition=hadoop -DskipTests -B
+   save /target/bin/*.zip, make "mvn clean" and restore saved files before deploying fabric.
 
-   mvn deploy -P apache-release,gpg,release,scala,lgpl,deploy-ignite-site -Dignite.edition=fabric -DskipTests -B
+   Binary artifact name can be changed by setting additional property -Dignite.zip.pattern. Binary artifact will be
+   created inside /target/bin folder when release profile is used.
 
-   Binary artifact name can be changed by setting additional property -Dignite.zip.pattern.
-   Sources package name is fixed. Sources package zip will be created automatically when apache-release profile used.
+   Sources artifact name is fixed. Sources artifact will be created inside /target dir when apache-release profile is used.
 
-   In case you want to release both fabric and hadoop editions you have to build hadoop first, save /target/bin/*.zip,
-   make "mvn clean" and restore them before deploy step.
+   Nexus staging (repository.apache.org) should be closed with appropriate comment contains release version and
+   release candidate number, for example "Apache Ignite 1.0.0-rc7", when mvn deploy finished.
 
-   Nexus staging (repository.apache.org) should be closed when mvn deploy finished.
-   Checkout https://dist.apache.org/repos/dist/dev/incubator/ignite svn. Create new folder with name equals to released
-   version at svn root. Copy target/site folder content to svn/ignite/<version> folder and commit.
+   Checkout https://dist.apache.org/repos/dist/dev/incubator/ignite svn. Create release candidate folder with name
+   equals to release version with "-rc*" ending, for example "1.0.0-rc7", at svn root.
+   Copy /target/site folder content to svn/ignite/<rc-version> folder and commit with appropriate comment.
 
-Start vote based on https://dist.apache.org/repos/dist/dev/incubator/ignite/<version>.
+   Make appropriate git tag for release candidate, for example "ignite-1.0.0-incubating-rc7".
 
-Release nexus staging and copy binaries and sources from https://dist.apache.org/repos/dist/dev/incubator/ignite/<version>
+Start vote based on https://dist.apache.org/repos/dist/dev/incubator/ignite/<rc-version>.
+
+Release nexus staging, move binaries and sources from https://dist.apache.org/repos/dist/dev/incubator/ignite/<rc-version>
 to https://dist.apache.org/repos/dist/release/incubator/ignite/<version> when version accepted.
+Use svn mv ^/dev/incubator/ignite/<rc-version> ^/release/incubator/ignite/<version> command for proper moving.
+
+Make appropriate git tag for released version, for example "ignite-1.0.0-incubating".
+
+Send an email to dev@ignite.incubator.apache.org contains release svn url.
 
 
 JCache TCK compliance


[32/42] incubator-ignite git commit: # ignite-883 fix ConcurrentModificationException during DiscoCache initialization from the GridDiscoveryManager.start

Posted by ag...@apache.org.
# ignite-883 fix ConcurrentModificationException during DiscoCache initialization from the GridDiscoveryManager.start


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

Branch: refs/heads/ignite-389
Commit: f4da46cd25b61b3d7b55b6fba3884c8fa05f5cb8
Parents: 730ef10
Author: sboikov <sb...@gridgain.com>
Authored: Mon Jun 8 10:27:04 2015 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Mon Jun 8 10:27:04 2015 +0300

----------------------------------------------------------------------
 .../ignite/internal/managers/discovery/GridDiscoveryManager.java   | 2 +-
 1 file changed, 1 insertion(+), 1 deletion(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/f4da46cd/modules/core/src/main/java/org/apache/ignite/internal/managers/discovery/GridDiscoveryManager.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/managers/discovery/GridDiscoveryManager.java b/modules/core/src/main/java/org/apache/ignite/internal/managers/discovery/GridDiscoveryManager.java
index 142dbaa..5983553 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/managers/discovery/GridDiscoveryManager.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/managers/discovery/GridDiscoveryManager.java
@@ -173,7 +173,7 @@ public class GridDiscoveryManager extends GridManagerAdapter<DiscoverySpi> {
         new ConcurrentHashMap8<>();
 
     /** Map of dynamic cache filters. */
-    private Map<String, CachePredicate> registeredCaches = new HashMap<>();
+    private Map<String, CachePredicate> registeredCaches = new ConcurrentHashMap<>();
 
     /** */
     private final GridSpinBusyLock busyLock = new GridSpinBusyLock();


[02/42] incubator-ignite git commit: # IGNITE-883 Set named thread factory to pool. (cherry picked from commit 0ac2ff2)

Posted by ag...@apache.org.
# IGNITE-883 Set named thread factory to pool.
(cherry picked from commit 0ac2ff2)


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

Branch: refs/heads/ignite-389
Commit: dce67896a81c6a2148e53284aef49e5c1aa4d3b7
Parents: 97d0b04
Author: sevdokimov <se...@gridgain.com>
Authored: Mon Jun 1 18:01:34 2015 +0300
Committer: sevdokimov <se...@gridgain.com>
Committed: Mon Jun 1 19:48:13 2015 +0300

----------------------------------------------------------------------
 .../ignite/internal/processors/service/GridServiceProcessor.java | 4 +++-
 1 file changed, 3 insertions(+), 1 deletion(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/dce67896/modules/core/src/main/java/org/apache/ignite/internal/processors/service/GridServiceProcessor.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/service/GridServiceProcessor.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/service/GridServiceProcessor.java
index 2e31b69..64eb1c1 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/service/GridServiceProcessor.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/service/GridServiceProcessor.java
@@ -70,7 +70,7 @@ public class GridServiceProcessor extends GridProcessorAdapter {
     private final ConcurrentMap<String, GridFutureAdapter<?>> undepFuts = new ConcurrentHashMap8<>();
 
     /** Deployment executor service. */
-    private final ExecutorService depExe = Executors.newSingleThreadExecutor();
+    private final ExecutorService depExe;
 
     /** Busy lock. */
     private final GridSpinBusyLock busyLock = new GridSpinBusyLock();
@@ -98,6 +98,8 @@ public class GridServiceProcessor extends GridProcessorAdapter {
      */
     public GridServiceProcessor(GridKernalContext ctx) {
         super(ctx);
+
+        depExe = Executors.newSingleThreadExecutor(new IgniteThreadFactory(ctx.gridName(), "srvc-deploy"));
     }
 
     /** {@inheritDoc} */


[25/42] incubator-ignite git commit: # ignite-883

Posted by ag...@apache.org.
# ignite-883


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

Branch: refs/heads/ignite-389
Commit: e237b00d84039383723e00929b639fa62be03de7
Parents: fb827a77
Author: sboikov <sb...@gridgain.com>
Authored: Fri Jun 5 16:12:46 2015 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Fri Jun 5 18:10:34 2015 +0300

----------------------------------------------------------------------
 .../util/nio/GridNioRecoveryDescriptor.java     | 11 ++++++---
 .../ignite/spi/discovery/tcp/ServerImpl.java    |  9 ++++---
 .../tcp/TcpClientDiscoverySpiSelfTest.java      | 25 ++++++++++++++++++++
 3 files changed, 37 insertions(+), 8 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/e237b00d/modules/core/src/main/java/org/apache/ignite/internal/util/nio/GridNioRecoveryDescriptor.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/util/nio/GridNioRecoveryDescriptor.java b/modules/core/src/main/java/org/apache/ignite/internal/util/nio/GridNioRecoveryDescriptor.java
index 08a9937..3bc81ea 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/util/nio/GridNioRecoveryDescriptor.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/util/nio/GridNioRecoveryDescriptor.java
@@ -176,9 +176,11 @@ public class GridNioRecoveryDescriptor {
         while (acked < rcvCnt) {
             GridNioFuture<?> fut = msgFuts.pollFirst();
 
-            assert fut != null;
+            assert fut != null : "Missed message future [rcvCnt=" + rcvCnt +
+                ", acked=" + acked +
+                ", desc=" + this + ']';
 
-            assert fut.isDone();
+            assert fut.isDone() : fut;
 
             acked++;
         }
@@ -239,7 +241,10 @@ public class GridNioRecoveryDescriptor {
      * @param rcvCnt Number of messages received by remote node.
      */
     public void onHandshake(long rcvCnt) {
-        ackReceived(rcvCnt);
+        synchronized (this) {
+            if (!nodeLeft)
+                ackReceived(rcvCnt);
+        }
 
         resendCnt = msgFuts.size();
     }

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/e237b00d/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/ServerImpl.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/ServerImpl.java b/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/ServerImpl.java
index 302b721..0270a7c 100644
--- a/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/ServerImpl.java
+++ b/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/ServerImpl.java
@@ -1752,7 +1752,7 @@ class ServerImpl extends TcpDiscoveryImpl {
         @Nullable Collection<TcpDiscoveryAbstractMessage> messages(IgniteUuid lastMsgId) {
             assert lastMsgId != null;
 
-            Collection<TcpDiscoveryAbstractMessage> copy = new ArrayList<>(msgs.size());
+            Collection<TcpDiscoveryAbstractMessage> cp = new ArrayList<>(msgs.size());
 
             boolean skip = true;
 
@@ -1762,10 +1762,10 @@ class ServerImpl extends TcpDiscoveryImpl {
                         skip = false;
                 }
                 else
-                    copy.add(msg);
+                    cp.add(msg);
             }
 
-            return !skip ? copy : null;
+            return !skip ? cp : null;
         }
 
         /**
@@ -2704,8 +2704,7 @@ class ServerImpl extends TcpDiscoveryImpl {
                 node.aliveCheck(spi.maxMissedClientHbs);
 
                 if (isLocalNodeCoordinator()) {
-                    Collection<TcpDiscoveryAbstractMessage> pending =
-                        pendingMsgs.messages(msg.lastMessageId());
+                    Collection<TcpDiscoveryAbstractMessage> pending = pendingMsgs.messages(msg.lastMessageId());
 
                     if (pending != null) {
                         msg.pendingMessages(pending);

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/e237b00d/modules/core/src/test/java/org/apache/ignite/spi/discovery/tcp/TcpClientDiscoverySpiSelfTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/spi/discovery/tcp/TcpClientDiscoverySpiSelfTest.java b/modules/core/src/test/java/org/apache/ignite/spi/discovery/tcp/TcpClientDiscoverySpiSelfTest.java
index f7be340..7333020 100644
--- a/modules/core/src/test/java/org/apache/ignite/spi/discovery/tcp/TcpClientDiscoverySpiSelfTest.java
+++ b/modules/core/src/test/java/org/apache/ignite/spi/discovery/tcp/TcpClientDiscoverySpiSelfTest.java
@@ -469,6 +469,31 @@ public class TcpClientDiscoverySpiSelfTest extends GridCommonAbstractTest {
     /**
      * @throws Exception If failed.
      */
+    public void testClientReconnectOneServerOneClient() throws Exception {
+        clientsPerSrv = 1;
+
+        startServerNodes(1);
+        startClientNodes(1);
+
+        checkNodes(1, 1);
+
+        srvLeftLatch = new CountDownLatch(1);
+        srvFailedLatch = new CountDownLatch(1);
+
+        attachListeners(1, 0);
+
+        ((TcpDiscoverySpi)G.ignite("client-0").configuration().getDiscoverySpi()).brakeConnection();
+
+        assertFalse(srvFailedLatch.await(2000, TimeUnit.MILLISECONDS));
+
+        assertEquals(1L, srvLeftLatch.getCount());
+
+        checkNodes(1, 1);
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
     public void testGetMissedMessagesOnReconnect() throws Exception {
         clientsPerSrv = 1;
 


[12/42] incubator-ignite git commit: Merge branches 'ignite-gg-9621' and 'ignite-sprint-5' of https://git-wip-us.apache.org/repos/asf/incubator-ignite into ignite-gg-9621

Posted by ag...@apache.org.
Merge branches 'ignite-gg-9621' and 'ignite-sprint-5' of https://git-wip-us.apache.org/repos/asf/incubator-ignite into ignite-gg-9621

Conflicts:
	modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCache.java
	modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCacheMetrics.java


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

Branch: refs/heads/ignite-389
Commit: 1f516a2531bec313bb6102f79d43d75e04c36a48
Parents: 1497cbe bf3203a
Author: AKuznetsov <ak...@gridgain.com>
Authored: Thu Jun 4 16:34:33 2015 +0700
Committer: AKuznetsov <ak...@gridgain.com>
Committed: Thu Jun 4 16:34:33 2015 +0700

----------------------------------------------------------------------
 .gitignore                                      |    3 +-
 DEVNOTES.txt                                    |   47 +-
 LICENSE                                         |  238 +
 LICENSE.txt                                     |  238 -
 NOTICE                                          |   12 +
 NOTICE.txt                                      |   12 -
 RELEASE_NOTES.txt                               |   13 +-
 assembly/dependencies-fabric.xml                |   50 +
 assembly/dependencies-hadoop.xml                |    1 -
 assembly/dependencies-optional-fabric.xml       |   82 -
 assembly/dependencies-optional-hadoop.xml       |   83 -
 assembly/dependencies-visor-console.xml         |    3 +
 assembly/release-base.xml                       |   15 +-
 assembly/release-fabric.xml                     |    5 +
 assembly/release-schema-import.xml              |   50 +
 bin/ignite-schema-import.bat                    |    2 +-
 bin/ignite-schema-import.sh                     |    2 +-
 bin/ignite.bat                                  |    2 +-
 bin/ignite.sh                                   |    2 +-
 bin/ignitevisorcmd.bat                          |    2 +-
 bin/ignitevisorcmd.sh                           |    2 +-
 bin/include/build-classpath.bat                 |   46 +
 bin/include/build-classpath.sh                  |   71 +
 bin/include/functions.sh                        |    2 +-
 bin/include/target-classpath.bat                |   46 -
 bin/include/target-classpath.sh                 |   71 -
 dev-tools/.gitignore                            |    2 +
 dev-tools/build.gradle                          |   54 +
 .../gradle/wrapper/gradle-wrapper.properties    |   20 +
 dev-tools/gradlew                               |  233 +
 dev-tools/slurp.sh                              |   76 +
 dev-tools/src/main/groovy/jiraslurp.groovy      |  544 ++
 examples/README.txt                             |    2 +-
 examples/pom.xml                                |    2 +-
 .../hibernate/CacheHibernatePersonStore.java    |  202 +-
 .../hibernate/CacheHibernateStoreExample.java   |   20 +-
 .../store/jdbc/CacheJdbcPersonStore.java        |  180 +-
 .../store/jdbc/CacheJdbcStoreExample.java       |   16 +-
 .../store/spring/CacheSpringPersonStore.java    |  128 +
 .../store/spring/CacheSpringStoreExample.java   |  143 +
 .../datagrid/store/spring/package-info.java     |   22 +
 .../streaming/StreamTransformerExample.java     |    4 +-
 .../streaming/StreamVisitorExample.java         |    4 +-
 .../ignite/examples/streaming/package-info.java |    1 -
 .../streaming/wordcount/CacheConfig.java        |    7 +-
 .../streaming/wordcount/QueryWords.java         |   12 +-
 .../streaming/wordcount/StreamWords.java        |   12 +-
 .../streaming/wordcount/package-info.java       |    1 -
 .../socket/WordsSocketStreamerClient.java       |   82 +
 .../socket/WordsSocketStreamerServer.java       |  124 +
 .../wordcount/socket/package-info.java          |   21 +
 .../examples/ScalarContinuationExample.scala    |   10 +-
 modules/aop/pom.xml                             |    2 +-
 .../aop/aspectj/GridifyAspectJAspect.java       |    2 +-
 .../aspectj/GridifySetToSetAspectJAspect.java   |    2 +-
 .../aspectj/GridifySetToValueAspectJAspect.java |    2 +-
 .../aop/spring/GridifySetToSetSpringAspect.java |    2 +-
 .../spring/GridifySetToValueSpringAspect.java   |    2 +-
 .../gridify/aop/spring/GridifySpringAspect.java |    2 +-
 modules/aws/pom.xml                             |    2 +-
 .../spi/checkpoint/s3/S3CheckpointSpi.java      |    2 +-
 .../s3/S3CheckpointManagerSelfTest.java         |    2 +-
 .../checkpoint/s3/S3CheckpointSpiSelfTest.java  |    2 +-
 .../s3/S3SessionCheckpointSelfTest.java         |    2 +-
 .../config/grid-client-config.properties        |   50 +-
 modules/clients/pom.xml                         |    2 +-
 .../ClientPropertiesConfigurationSelfTest.java  |   12 +-
 .../ClientAbstractConnectivitySelfTest.java     |   14 +
 .../clients/src/test/resources/spring-cache.xml |    4 +-
 .../src/test/resources/spring-server-node.xml   |    4 +-
 .../test/resources/spring-server-ssl-node.xml   |    4 +-
 modules/cloud/README.txt                        |   32 +
 modules/cloud/licenses/apache-2.0.txt           |  202 +
 modules/cloud/pom.xml                           |  106 +
 .../cloud/TcpDiscoveryCloudIpFinder.java        |  433 ++
 .../tcp/ipfinder/cloud/package-info.java        |   21 +
 .../TcpDiscoveryCloudIpFinderSelfTest.java      |  122 +
 .../tcp/ipfinder/cloud/package-info.java        |   22 +
 .../ignite/testsuites/IgniteCloudTestSuite.java |  112 +
 modules/codegen/pom.xml                         |   14 +-
 .../ignite/codegen/MessageCodeGenerator.java    |   30 +-
 modules/core/pom.xml                            |    2 +-
 modules/core/src/main/java/META-INF/LICENSE     |  238 +
 modules/core/src/main/java/META-INF/NOTICE      |   12 +
 .../src/main/java/org/apache/ignite/Ignite.java |    8 +-
 .../java/org/apache/ignite/IgniteCache.java     |    5 +
 .../org/apache/ignite/IgniteJdbcDriver.java     |   81 +-
 .../java/org/apache/ignite/IgniteLogger.java    |    8 +-
 .../java/org/apache/ignite/IgniteServices.java  |    7 +-
 .../apache/ignite/IgniteSystemProperties.java   |    6 -
 .../main/java/org/apache/ignite/Ignition.java   |   64 +-
 .../apache/ignite/cache/CacheInterceptor.java   |    9 +-
 .../org/apache/ignite/cache/CacheMetrics.java   |  187 +-
 .../cache/CacheServerNotFoundException.java     |   12 +-
 .../apache/ignite/cache/CachingProvider.java    |    3 +
 .../ignite/cache/eviction/EvictableEntry.java   |    7 +
 .../ignite/cache/eviction/EvictionPolicy.java   |    2 +
 .../cache/eviction/fifo/FifoEvictionPolicy.java |  120 +-
 .../eviction/fifo/FifoEvictionPolicyMBean.java  |   22 +
 .../igfs/IgfsPerBlockLruEvictionPolicy.java     |    3 +-
 .../cache/eviction/lru/LruEvictionPolicy.java   |  136 +-
 .../eviction/lru/LruEvictionPolicyMBean.java    |   38 +
 .../eviction/random/RandomEvictionPolicy.java   |   10 +-
 .../eviction/sorted/SortedEvictionPolicy.java   |  520 ++
 .../sorted/SortedEvictionPolicyMBean.java       |   88 +
 .../cache/eviction/sorted/package-info.java     |   21 +
 .../apache/ignite/cache/query/QueryMetrics.java |    6 +-
 .../apache/ignite/cache/store/CacheStore.java   |    2 +
 .../ignite/cache/store/CacheStoreSession.java   |   22 +
 .../cache/store/CacheStoreSessionListener.java  |  133 +
 .../jdbc/CacheJdbcStoreSessionListener.java     |  141 +
 .../org/apache/ignite/cluster/ClusterNode.java  |   24 +-
 .../ignite/compute/ComputeJobContinuation.java  |    2 +
 .../configuration/CacheConfiguration.java       |  350 +-
 .../configuration/ConnectorConfiguration.java   |    2 +-
 .../configuration/IgniteConfiguration.java      |  493 +-
 .../ignite/configuration/TopologyValidator.java |   35 +
 .../ignite/events/CacheQueryExecutedEvent.java  |    3 +-
 .../ignite/events/CacheQueryReadEvent.java      |    3 +-
 .../org/apache/ignite/igfs/IgfsUserContext.java |  119 +
 .../igfs/secondary/IgfsSecondaryFileSystem.java |    7 +
 .../ignite/internal/ClusterMetricsSnapshot.java |   14 +
 .../ignite/internal/GridDirectCollection.java   |    3 +
 .../internal/GridEventConsumeHandler.java       |  126 +-
 .../ignite/internal/GridJobContextImpl.java     |   99 +-
 .../ignite/internal/GridKernalContext.java      |    5 +
 .../ignite/internal/GridKernalContextImpl.java  |    5 +
 .../ignite/internal/GridUpdateNotifier.java     |   66 +-
 .../ignite/internal/IgniteComponentType.java    |   36 +-
 .../org/apache/ignite/internal/IgniteEx.java    |   10 +-
 .../apache/ignite/internal/IgniteKernal.java    |  164 +-
 .../org/apache/ignite/internal/IgnitionEx.java  |  234 +-
 .../ignite/internal/MarshallerContextImpl.java  |    4 +-
 .../client/GridClientConfiguration.java         |    2 +-
 .../connection/GridClientNioTcpConnection.java  |    9 +-
 .../internal/cluster/ClusterGroupAdapter.java   |   16 +
 .../ClusterTopologyServerNotFoundException.java |   12 +-
 .../internal/direct/DirectByteBufferStream.java |    4 +-
 .../internal/events/DiscoveryCustomEvent.java   |   18 +-
 .../internal/igfs/common/IgfsMarshaller.java    |   35 +-
 .../igfs/common/IgfsPathControlRequest.java     |   22 +
 .../interop/InteropAwareEventFilter.java        |   37 +
 .../internal/interop/InteropBootstrap.java      |   34 +
 .../interop/InteropBootstrapFactory.java        |   39 +
 .../internal/interop/InteropException.java      |   71 +
 .../internal/interop/InteropIgnition.java       |  166 +
 .../interop/InteropLocalEventListener.java      |   28 +
 .../interop/InteropNoCallbackException.java     |   50 +
 .../internal/interop/InteropProcessor.java      |   36 +
 .../internal/managers/GridManagerAdapter.java   |   63 +-
 .../managers/communication/GridIoManager.java   |   71 +-
 .../communication/GridIoMessageFactory.java     |   12 +-
 .../GridLifecycleAwareMessageFilter.java        |   38 +
 .../deployment/GridDeploymentClassLoader.java   |    2 +-
 .../deployment/GridDeploymentManager.java       |    2 +-
 .../GridDeploymentPerVersionStore.java          |    3 +-
 .../managers/discovery/CustomEventListener.java |   31 +
 .../discovery/CustomMessageWrapper.java         |   63 +
 .../discovery/DiscoveryCustomMessage.java       |   48 +
 .../discovery/GridDiscoveryManager.java         |  289 +-
 .../eventstorage/GridEventStorageManager.java   |   35 +-
 .../managers/indexing/GridIndexingManager.java  |   14 +-
 .../affinity/GridAffinityAssignmentCache.java   |   43 +-
 .../processors/cache/CacheEntryImpl.java        |   29 +-
 .../cache/CacheEvictableEntryImpl.java          |   31 +
 .../processors/cache/CacheInvokeResult.java     |   24 +-
 .../processors/cache/CacheLockImpl.java         |   20 +-
 .../processors/cache/CacheMetricsImpl.java      |  367 +-
 .../cache/CacheMetricsMXBeanImpl.java           |  100 +
 .../processors/cache/CacheMetricsSnapshot.java  |  380 +-
 .../processors/cache/CacheObjectImpl.java       |    2 +-
 .../processors/cache/CacheOperationContext.java |  170 +
 .../processors/cache/CacheProjection.java       | 1386 ----
 .../cache/CacheStoreBalancingWrapper.java       |    6 +
 .../cache/CacheVersionedEntryImpl.java          |   29 +-
 .../cache/DynamicCacheChangeBatch.java          |   20 +-
 .../cache/DynamicCacheDescriptor.java           |   18 +-
 .../internal/processors/cache/GridCache.java    |  223 -
 .../processors/cache/GridCacheAdapter.java      |  821 +-
 .../cache/GridCacheAffinityManager.java         |   26 +-
 .../processors/cache/GridCacheAtomicFuture.java |    7 -
 .../cache/GridCacheConcurrentMap.java           |   45 +-
 .../processors/cache/GridCacheContext.java      |   82 +-
 .../processors/cache/GridCacheEntryEx.java      |   10 +
 .../cache/GridCacheEvictionManager.java         |   13 +-
 .../processors/cache/GridCacheGateway.java      |  121 +-
 .../processors/cache/GridCacheIoManager.java    |  330 +-
 .../processors/cache/GridCacheMapEntry.java     |  223 +-
 .../processors/cache/GridCacheMessage.java      |    8 +-
 .../processors/cache/GridCacheMvccManager.java  |   38 +-
 .../GridCachePartitionExchangeManager.java      |   83 +-
 .../processors/cache/GridCachePreloader.java    |    6 +-
 .../cache/GridCachePreloaderAdapter.java        |   11 +-
 .../processors/cache/GridCacheProcessor.java    |  289 +-
 .../processors/cache/GridCacheProjectionEx.java |  351 -
 .../cache/GridCacheProjectionImpl.java          |  766 --
 .../processors/cache/GridCacheProxy.java        |   27 -
 .../processors/cache/GridCacheProxyImpl.java    |  424 +-
 .../processors/cache/GridCacheReturn.java       |    5 +-
 .../cache/GridCacheSharedContext.java           |   50 +-
 .../processors/cache/GridCacheSwapManager.java  |  364 +-
 .../processors/cache/GridCacheTtlManager.java   |  156 +-
 .../processors/cache/GridCacheUtils.java        |  353 +-
 .../processors/cache/IgniteCacheProxy.java      |  465 +-
 .../processors/cache/IgniteInternalCache.java   | 1794 +++++
 .../cache/affinity/GridCacheAffinityImpl.java   |   12 +-
 .../cache/affinity/GridCacheAffinityProxy.java  |   30 +-
 .../CacheDataStructuresManager.java             |   10 +-
 ...ridCacheOptimisticCheckPreparedTxFuture.java |  383 -
 ...idCacheOptimisticCheckPreparedTxRequest.java |  232 -
 ...dCacheOptimisticCheckPreparedTxResponse.java |  179 -
 .../distributed/GridCacheTxRecoveryFuture.java  |  506 ++
 .../distributed/GridCacheTxRecoveryRequest.java |  261 +
 .../GridCacheTxRecoveryResponse.java            |  182 +
 .../GridDistributedCacheAdapter.java            |  210 +-
 .../distributed/GridDistributedCacheEntry.java  |    7 -
 .../distributed/GridDistributedLockRequest.java |  104 +-
 .../GridDistributedTxFinishRequest.java         |   70 +-
 .../distributed/GridDistributedTxMapping.java   |   22 +-
 .../GridDistributedTxPrepareRequest.java        |  112 +-
 .../GridDistributedTxRemoteAdapter.java         |   35 +-
 .../dht/GridClientPartitionTopology.java        |    8 +-
 .../dht/GridDhtAffinityAssignmentResponse.java  |   21 +-
 .../dht/GridDhtAssignmentFetchFuture.java       |    4 +-
 .../distributed/dht/GridDhtCacheAdapter.java    |   57 +-
 .../distributed/dht/GridDhtCacheEntry.java      |    6 +-
 .../cache/distributed/dht/GridDhtGetFuture.java |   20 +-
 .../distributed/dht/GridDhtLocalPartition.java  |    2 +-
 .../distributed/dht/GridDhtLockFuture.java      |   50 +-
 .../distributed/dht/GridDhtLockRequest.java     |   50 +-
 .../dht/GridDhtOffHeapCacheEntry.java           |   63 +
 .../dht/GridDhtPartitionTopologyImpl.java       |   30 +-
 .../distributed/dht/GridDhtTopologyFuture.java  |    8 +
 .../dht/GridDhtTransactionalCacheAdapter.java   |  256 +-
 .../distributed/dht/GridDhtTxFinishFuture.java  |  105 +-
 .../distributed/dht/GridDhtTxFinishRequest.java |   43 +-
 .../cache/distributed/dht/GridDhtTxLocal.java   |   45 +-
 .../distributed/dht/GridDhtTxLocalAdapter.java  |  152 +-
 .../cache/distributed/dht/GridDhtTxMapping.java |    2 +-
 .../distributed/dht/GridDhtTxPrepareFuture.java |  115 +-
 .../dht/GridDhtTxPrepareRequest.java            |   60 +-
 .../cache/distributed/dht/GridDhtTxRemote.java  |   22 +-
 .../distributed/dht/GridNoStorageCacheMap.java  |    4 +-
 .../dht/GridPartitionedGetFuture.java           |   29 +-
 .../dht/atomic/GridDhtAtomicCache.java          |  151 +-
 .../atomic/GridDhtAtomicOffHeapCacheEntry.java  |   63 +
 .../dht/atomic/GridDhtAtomicUpdateFuture.java   |   27 +-
 .../dht/atomic/GridDhtAtomicUpdateResponse.java |    8 +
 .../dht/atomic/GridNearAtomicUpdateFuture.java  |  165 +-
 .../dht/atomic/GridNearAtomicUpdateRequest.java |  157 +-
 .../atomic/GridNearAtomicUpdateResponse.java    |   18 +-
 .../dht/colocated/GridDhtColocatedCache.java    |   78 +-
 .../colocated/GridDhtColocatedLockFuture.java   |  250 +-
 .../GridDhtColocatedOffHeapCacheEntry.java      |   63 +
 .../colocated/GridDhtDetachedCacheEntry.java    |    4 +-
 .../dht/preloader/GridDhtForceKeysFuture.java   |   10 +-
 .../dht/preloader/GridDhtForceKeysResponse.java |   54 +-
 .../preloader/GridDhtPartitionDemandPool.java   |   26 +-
 .../dht/preloader/GridDhtPartitionMap.java      |    2 +-
 .../preloader/GridDhtPartitionSupplyPool.java   |   31 +-
 .../GridDhtPartitionsExchangeFuture.java        |  390 +-
 .../preloader/GridDhtPartitionsFullMessage.java |    4 +-
 .../GridDhtPartitionsSingleMessage.java         |   33 +-
 .../dht/preloader/GridDhtPreloader.java         |   39 +-
 .../preloader/GridDhtPreloaderAssignments.java  |    3 +-
 .../distributed/near/GridNearAtomicCache.java   |   18 +-
 .../distributed/near/GridNearCacheAdapter.java  |   25 +-
 .../distributed/near/GridNearCacheEntry.java    |   22 +-
 .../distributed/near/GridNearGetFuture.java     |   39 +-
 .../distributed/near/GridNearGetResponse.java   |    8 +-
 .../distributed/near/GridNearLockFuture.java    |  319 +-
 .../distributed/near/GridNearLockRequest.java   |   52 +-
 .../distributed/near/GridNearLockResponse.java  |   48 +-
 .../near/GridNearOffHeapCacheEntry.java         |   60 +
 .../near/GridNearOptimisticTxPrepareFuture.java |  815 ++
 .../GridNearPessimisticTxPrepareFuture.java     |  350 +
 .../near/GridNearTransactionalCache.java        |   57 +-
 .../near/GridNearTxFinishFuture.java            |    3 +
 .../near/GridNearTxFinishRequest.java           |   28 +-
 .../cache/distributed/near/GridNearTxLocal.java |  163 +-
 .../near/GridNearTxPrepareFuture.java           | 1030 ---
 .../near/GridNearTxPrepareFutureAdapter.java    |  231 +
 .../near/GridNearTxPrepareRequest.java          |   86 +-
 .../near/GridNearTxPrepareResponse.java         |   92 +-
 .../distributed/near/GridNearTxRemote.java      |   31 +-
 .../processors/cache/local/GridLocalCache.java  |   14 +-
 .../cache/local/GridLocalCacheEntry.java        |   18 +
 .../local/atomic/GridLocalAtomicCache.java      |  133 +-
 .../processors/cache/query/CacheQueries.java    |  143 -
 .../processors/cache/query/CacheQuery.java      |   12 +-
 .../query/GridCacheDistributedQueryManager.java |    3 +
 .../cache/query/GridCacheLocalQueryFuture.java  |    3 +
 .../cache/query/GridCacheQueriesEx.java         |   68 -
 .../cache/query/GridCacheQueriesImpl.java       |  217 -
 .../cache/query/GridCacheQueriesProxy.java      |  285 -
 .../cache/query/GridCacheQueryAdapter.java      |   24 +-
 .../cache/query/GridCacheQueryErrorFuture.java  |    2 +
 .../query/GridCacheQueryFutureAdapter.java      |    3 +
 .../cache/query/GridCacheQueryManager.java      |  173 +-
 .../cache/query/GridCacheSqlQuery.java          |  137 +-
 .../cache/query/GridCacheTwoStepQuery.java      |   25 +-
 .../continuous/CacheContinuousQueryManager.java |   28 +-
 .../jdbc/GridCacheQueryJdbcMetadataTask.java    |    7 +-
 .../cache/store/CacheOsStoreManager.java        |    1 -
 .../cache/store/CacheStoreManager.java          |    7 +-
 .../store/GridCacheStoreManagerAdapter.java     |  202 +-
 .../cache/transactions/IgniteInternalTx.java    |   27 +-
 .../transactions/IgniteTransactionsImpl.java    |    4 +-
 .../cache/transactions/IgniteTxAdapter.java     |  135 +-
 .../cache/transactions/IgniteTxEntry.java       |   66 +-
 .../cache/transactions/IgniteTxHandler.java     |  330 +-
 .../transactions/IgniteTxLocalAdapter.java      |  439 +-
 .../cache/transactions/IgniteTxLocalEx.java     |   25 +-
 .../cache/transactions/IgniteTxManager.java     |  416 +-
 .../cacheobject/IgniteCacheObjectProcessor.java |    5 +-
 .../IgniteCacheObjectProcessorImpl.java         |    4 +-
 .../closure/GridClosureProcessor.java           |   16 +-
 .../continuous/AbstractContinuousMessage.java   |   54 +
 .../continuous/GridContinuousMessageType.java   |   12 -
 .../continuous/GridContinuousProcessor.java     |  836 +-
 .../processors/continuous/StartRequestData.java |  267 +
 .../StartRoutineAckDiscoveryMessage.java        |   63 +
 .../StartRoutineDiscoveryMessage.java           |   85 +
 .../StopRoutineAckDiscoveryMessage.java         |   49 +
 .../continuous/StopRoutineDiscoveryMessage.java |   49 +
 .../datastreamer/DataStreamProcessor.java       |   28 +-
 .../datastreamer/DataStreamerCacheUpdaters.java |   15 +-
 .../datastreamer/DataStreamerImpl.java          |   86 +-
 .../datastreamer/DataStreamerRequest.java       |   38 +-
 .../datastreamer/DataStreamerUpdateJob.java     |   16 +-
 .../datastructures/DataStructuresProcessor.java |   24 +-
 .../datastructures/GridCacheAtomicLongImpl.java |    4 +-
 .../GridCacheAtomicReferenceImpl.java           |    4 +-
 .../GridCacheAtomicSequenceImpl.java            |    4 +-
 .../GridCacheAtomicStampedImpl.java             |    4 +-
 .../GridCacheCountDownLatchImpl.java            |    4 +-
 .../datastructures/GridCacheSetImpl.java        |    2 +-
 .../dr/IgniteDrDataStreamerCacheUpdater.java    |    2 -
 .../internal/processors/hadoop/HadoopJob.java   |    2 +-
 .../processors/igfs/IgfsDataManager.java        |    9 +-
 .../processors/igfs/IgfsDeleteWorker.java       |    4 +
 .../ignite/internal/processors/igfs/IgfsEx.java |    8 +-
 .../processors/igfs/IgfsFileWorkerBatch.java    |    3 +
 .../internal/processors/igfs/IgfsImpl.java      |    8 +-
 .../processors/igfs/IgfsIpcHandler.java         |  184 +-
 .../processors/igfs/IgfsMetaManager.java        |   10 +-
 .../igfs/IgfsSecondaryFileSystemImpl.java       |    9 +-
 .../internal/processors/igfs/IgfsServer.java    |    4 +-
 .../internal/processors/igfs/IgfsThread.java    |    8 +-
 .../internal/processors/igfs/IgfsUtils.java     |   27 +-
 .../processors/job/GridJobHoldListener.java     |    6 +-
 .../processors/job/GridJobProcessor.java        |   28 +-
 .../internal/processors/job/GridJobWorker.java  |   32 +-
 .../offheap/GridOffHeapProcessor.java           |   17 +
 .../portable/GridPortableInputStream.java       |   26 -
 .../processors/query/GridQueryIndexing.java     |   39 +-
 .../processors/query/GridQueryProcessor.java    |  418 +-
 .../messages/GridQueryNextPageResponse.java     |   68 +-
 .../h2/twostep/messages/GridQueryRequest.java   |   21 +-
 .../processors/resource/GridResourceField.java  |   16 +-
 .../processors/resource/GridResourceIoc.java    |  389 +-
 .../processors/resource/GridResourceMethod.java |   13 +
 .../resource/GridResourceProcessor.java         |   20 +-
 .../processors/rest/GridRestCommand.java        |   11 +-
 .../processors/rest/GridRestProcessor.java      |   12 +-
 .../message/GridClientCacheQueryRequest.java    |  366 -
 .../cache/GridCacheClientQueryResult.java       |   97 -
 .../handlers/cache/GridCacheCommandHandler.java |   64 +-
 .../cache/GridCacheQueryCommandHandler.java     |  480 --
 .../top/GridTopologyCommandHandler.java         |    3 +-
 .../protocols/tcp/GridTcpRestNioListener.java   |   24 -
 .../rest/request/GridRestCacheQueryRequest.java |  143 -
 .../service/GridServiceProcessor.java           |  132 +-
 .../processors/task/GridTaskProcessor.java      |    6 +-
 .../processors/task/GridTaskWorker.java         |   12 +
 .../timeout/GridTimeoutProcessor.java           |    3 +
 .../ignite/internal/util/GridJavaProcess.java   |   26 +-
 .../ignite/internal/util/IgniteUtils.java       |   50 +-
 .../internal/util/future/GridFutureAdapter.java |    4 +-
 .../util/ipc/loopback/IpcServerTcpEndpoint.java |    2 +-
 .../shmem/IpcSharedMemoryServerEndpoint.java    |    4 +-
 .../util/lang/GridComputeJobWrapper.java        |   96 -
 .../util/lang/GridFilteredIterator.java         |    2 +-
 .../ignite/internal/util/lang/GridFunc.java     | 7218 +++++-------------
 .../internal/util/nio/GridBufferedParser.java   |    4 -
 .../internal/util/nio/GridDelimitedParser.java  |   91 +
 .../util/nio/GridNioDelimitedBuffer.java        |  106 +
 .../ignite/internal/util/nio/GridNioServer.java |    6 +
 .../util/offheap/GridOffHeapPartitionedMap.java |    9 +
 .../unsafe/GridUnsafePartitionedMap.java        |  155 +-
 .../util/spring/IgniteSpringHelper.java         |   56 +-
 .../util/tostring/GridToStringBuilder.java      |    2 +-
 .../apache/ignite/internal/util/typedef/X.java  |    2 +-
 .../ignite/internal/util/worker/GridWorker.java |    3 +
 .../ignite/internal/visor/cache/VisorCache.java |   92 +-
 .../visor/cache/VisorCacheConfiguration.java    |    7 -
 .../visor/cache/VisorCacheMetadataTask.java     |    9 +-
 .../internal/visor/cache/VisorCacheMetrics.java |   53 +-
 .../cache/VisorCacheNearConfiguration.java      |    4 +-
 .../visor/cache/VisorCacheNodesTask.java        |   74 +
 .../visor/cache/VisorCacheRebalanceTask.java    |    4 +-
 .../visor/cache/VisorCacheResetMetricsTask.java |    2 +-
 .../visor/cache/VisorCacheStartTask.java        |  155 +
 .../cache/VisorCacheStoreConfiguration.java     |   35 -
 .../visor/cache/VisorCacheSwapBackupsTask.java  |    2 +-
 .../cache/VisorCacheTypeFieldMetadata.java      |   36 +-
 .../visor/cache/VisorCacheTypeMetadata.java     |   99 +-
 .../internal/visor/igfs/VisorIgfsMetrics.java   |  128 +-
 .../visor/misc/VisorResolveHostNameTask.java    |    2 +-
 .../visor/node/VisorBasicConfiguration.java     |   11 +
 .../visor/node/VisorNodeDataCollectorJob.java   |    8 +-
 .../node/VisorNodeEventsCollectorTask.java      |   58 +-
 .../internal/visor/query/VisorQueryArg.java     |   31 +-
 .../internal/visor/query/VisorQueryCursor.java  |    1 -
 .../internal/visor/query/VisorQueryJob.java     |   13 +-
 .../internal/visor/query/VisorQueryTask.java    |   41 -
 .../internal/visor/util/VisorEventMapper.java   |   13 +
 .../internal/visor/util/VisorTaskUtils.java     |   12 +-
 .../apache/ignite/lang/IgniteAsyncSupport.java  |    4 +-
 .../apache/ignite/logger/java/JavaLogger.java   |   12 +-
 .../apache/ignite/marshaller/Marshaller.java    |   14 +-
 .../ignite/marshaller/jdk/JdkMarshaller.java    |   10 +-
 .../optimized/OptimizedMarshaller.java          |    8 +-
 .../ignite/messaging/MessagingListenActor.java  |    3 +
 .../ignite/mxbean/CacheMetricsMXBean.java       |   80 +
 .../apache/ignite/plugin/PluginProvider.java    |   26 +-
 .../apache/ignite/resources/LoggerResource.java |    2 +-
 .../apache/ignite/resources/SpringResource.java |    2 +-
 .../org/apache/ignite/services/Service.java     |    7 +-
 .../ignite/services/ServiceConfiguration.java   |    2 +-
 .../org/apache/ignite/spi/IgniteSpiAdapter.java |   61 +-
 .../org/apache/ignite/spi/IgniteSpiContext.java |   53 +-
 .../org/apache/ignite/spi/IgniteSpiThread.java  |    3 +
 .../checkpoint/cache/CacheCheckpointSpi.java    |    2 +-
 .../spi/checkpoint/jdbc/JdbcCheckpointSpi.java  |    2 +-
 .../sharedfs/SharedFsCheckpointSpi.java         |    4 +-
 .../fifoqueue/FifoQueueCollisionSpi.java        |   10 +-
 .../jobstealing/JobStealingCollisionSpi.java    |   14 +-
 .../PriorityQueueCollisionSpi.java              |    6 +-
 .../communication/tcp/TcpCommunicationSpi.java  |   86 +-
 .../tcp/TcpCommunicationSpiMBean.java           |    9 -
 .../ignite/spi/discovery/DiscoverySpi.java      |   27 +-
 .../discovery/DiscoverySpiCustomMessage.java    |   40 +
 .../spi/discovery/DiscoverySpiListener.java     |    5 +-
 .../ignite/spi/discovery/tcp/ClientImpl.java    | 1481 ++++
 .../ignite/spi/discovery/tcp/ServerImpl.java    | 4735 ++++++++++++
 .../discovery/tcp/TcpClientDiscoverySpi.java    | 1256 ---
 .../tcp/TcpClientDiscoverySpiMBean.java         |  164 -
 .../spi/discovery/tcp/TcpDiscoveryImpl.java     |  212 +
 .../spi/discovery/tcp/TcpDiscoverySpi.java      | 5792 +++-----------
 .../discovery/tcp/TcpDiscoverySpiAdapter.java   | 1156 ---
 .../spi/discovery/tcp/TcpDiscoverySpiMBean.java |    9 +
 .../tcp/internal/TcpDiscoveryNode.java          |    7 +-
 .../tcp/internal/TcpDiscoveryNodesRing.java     |    2 +-
 .../tcp/ipfinder/TcpDiscoveryIpFinder.java      |   10 +-
 .../TcpDiscoveryMulticastIpFinder.java          |   57 +-
 .../messages/TcpDiscoveryAbstractMessage.java   |   24 +-
 .../TcpDiscoveryClientHeartbeatMessage.java     |   67 +
 .../messages/TcpDiscoveryClientPingRequest.java |   56 +
 .../TcpDiscoveryClientPingResponse.java         |   67 +
 .../TcpDiscoveryCustomEventMessage.java         |   41 +-
 .../messages/TcpDiscoveryHeartbeatMessage.java  |   28 +-
 .../TcpDiscoveryNodeAddFinishedMessage.java     |   43 +
 .../messages/TcpDiscoveryNodeAddedMessage.java  |    2 +-
 .../tcp/messages/TcpDiscoveryPingRequest.java   |    6 +
 .../tcp/messages/TcpDiscoveryPingResponse.java  |   15 +-
 .../memory/MemoryEventStorageSpi.java           |   10 +-
 .../spi/failover/always/AlwaysFailoverSpi.java  |   10 +-
 .../jobstealing/JobStealingFailoverSpi.java     |    6 +-
 .../spi/failover/never/NeverFailoverSpi.java    |    8 +-
 .../apache/ignite/spi/indexing/IndexingSpi.java |    4 +-
 .../adaptive/AdaptiveLoadBalancingSpi.java      |   12 +-
 .../roundrobin/RoundRobinLoadBalancingSpi.java  |   10 +-
 .../WeightedRandomLoadBalancingSpi.java         |   10 +-
 .../spi/swapspace/file/FileSwapSpaceSpi.java    |   18 +-
 .../ignite/startup/BasicWarmupClosure.java      |   20 +-
 .../startup/cmdline/CommandLineStartup.java     |    5 +-
 .../startup/cmdline/CommandLineTransformer.java |    3 +
 .../org/apache/ignite/stream/StreamAdapter.java |  111 +
 .../ignite/stream/StreamTupleExtractor.java     |   33 +
 .../stream/socket/SocketMessageConverter.java   |   31 +
 .../ignite/stream/socket/SocketStreamer.java    |  218 +
 .../ignite/stream/socket/package-info.java      |   21 +
 .../TransactionSynchronization.java             |   45 -
 .../resources/META-INF/classnames.properties    |   65 +-
 .../core/src/main/resources/ignite.properties   |    2 +-
 .../src/test/config/load/merge-sort-base.xml    |    2 +-
 .../affinity/IgniteClientNodeAffinityTest.java  |  182 +
 ...cheStoreSessionListenerAbstractSelfTest.java |  315 +
 ...heStoreSessionListenerLifecycleSelfTest.java |  395 +
 .../CacheJdbcStoreSessionListenerSelfTest.java  |  175 +
 .../ignite/internal/GridAffinitySelfTest.java   |    1 +
 .../internal/GridCacheProjectionRemoveTest.java |   41 -
 .../internal/GridContinuousTaskSelfTest.java    |  114 +
 .../internal/GridDiscoveryEventSelfTest.java    |   35 +-
 ...ridFailFastNodeFailureDetectionSelfTest.java |  120 +
 .../internal/GridLifecycleBeanSelfTest.java     |   36 +
 .../internal/GridProjectionAbstractTest.java    |   16 +
 .../GridProjectionForCachesSelfTest.java        |   11 +-
 .../internal/GridReleaseTypeSelfTest.java       |   77 +-
 .../apache/ignite/internal/GridSelfTest.java    |    4 +-
 .../internal/GridUpdateNotifierSelfTest.java    |   30 +-
 .../internal/IgniteInternalCacheRemoveTest.java |   41 +
 .../GridDiscoveryManagerAliveCacheSelfTest.java |   66 +-
 .../GridDiscoveryManagerAttributesSelfTest.java |  122 +-
 .../discovery/GridDiscoveryManagerSelfTest.java |   46 +-
 .../GridAffinityProcessorAbstractSelfTest.java  |    1 +
 .../processors/cache/CacheGetFromJobTest.java   |  110 +
 .../cache/CacheOffheapMapEntrySelfTest.java     |  168 +
 .../cache/CacheRemoveAllSelfTest.java           |   81 +
 .../GridCacheAbstractFailoverSelfTest.java      |   56 +-
 .../cache/GridCacheAbstractFullApiSelfTest.java |  761 +-
 .../cache/GridCacheAbstractMetricsSelfTest.java |   48 +-
 .../GridCacheAbstractRemoveFailureTest.java     |   23 +
 .../cache/GridCacheAbstractSelfTest.java        |    6 +-
 .../GridCacheAtomicMessageCountSelfTest.java    |    1 +
 .../GridCacheConcurrentTxMultiNodeTest.java     |   10 +-
 ...idCacheConfigurationConsistencySelfTest.java |   14 +-
 ...CacheFullTextQueryMultithreadedSelfTest.java |    4 +-
 .../cache/GridCacheMemoryModeSelfTest.java      |   23 +-
 ...GridCacheMixedPartitionExchangeSelfTest.java |    2 +-
 .../processors/cache/GridCacheOffHeapTest.java  |    5 +-
 .../GridCachePreloadingEvictionsSelfTest.java   |    4 +-
 .../cache/GridCachePutAllFailoverSelfTest.java  |    3 +-
 .../processors/cache/GridCachePutAllTask.java   |   56 +-
 .../cache/GridCacheReloadSelfTest.java          |    6 +-
 .../GridCacheReturnValueTransferSelfTest.java   |    3 +
 .../cache/GridCacheSwapReloadSelfTest.java      |   20 +-
 ...acheTcpClientDiscoveryMultiThreadedTest.java |  190 +
 .../processors/cache/GridCacheTestEntryEx.java  |    6 +
 .../GridCacheVariableTopologySelfTest.java      |   12 +-
 ...ProjectionForCachesOnDaemonNodeSelfTest.java |  147 +
 .../IgniteCacheAbstractStopBusySelfTest.java    |    6 +-
 .../cache/IgniteCacheAbstractTest.java          |    2 +-
 .../IgniteCacheConfigurationTemplateTest.java   |    2 +-
 .../IgniteCacheEntryListenerAbstractTest.java   |    4 +-
 .../cache/IgniteCacheNearLockValueSelfTest.java |  148 +
 .../IgniteCacheP2pUnmarshallingErrorTest.java   |  200 +
 ...gniteCacheP2pUnmarshallingNearErrorTest.java |   59 +
 ...CacheP2pUnmarshallingRebalanceErrorTest.java |   80 +
 .../IgniteCacheP2pUnmarshallingTxErrorTest.java |  109 +
 .../IgniteCachePartitionMapUpdateTest.java      |  226 +
 .../cache/IgniteCachePeekModesAbstractTest.java |   20 +-
 .../cache/IgniteCachePutAllRestartTest.java     |  203 +
 .../cache/IgniteCacheTxPreloadNoWriteTest.java  |   29 +-
 .../cache/IgniteDynamicCacheStartSelfTest.java  |   86 +-
 .../IgniteDynamicClientCacheStartSelfTest.java  |  283 +
 .../cache/IgniteExchangeFutureHistoryTest.java  |   77 +
 .../cache/IgniteSystemCacheOnClientTest.java    |   97 +
 ...gniteTopologyValidatorAbstractCacheTest.java |  183 +
 ...iteTopologyValidatorAbstractTxCacheTest.java |  125 +
 ...ValidatorNearPartitionedAtomicCacheTest.java |   30 +
 ...logyValidatorNearPartitionedTxCacheTest.java |   30 +
 ...logyValidatorPartitionedAtomicCacheTest.java |   49 +
 ...TopologyValidatorPartitionedTxCacheTest.java |   30 +
 ...ologyValidatorReplicatedAtomicCacheTest.java |   49 +
 ...eTopologyValidatorReplicatedTxCacheTest.java |   30 +
 .../cache/IgniteTxMultiNodeAbstractTest.java    |   31 +-
 .../cache/OffHeapTieredTransactionSelfTest.java |  127 +
 .../GridCacheQueueApiSelfAbstractTest.java      |    4 +-
 .../IgniteClientDataStructuresAbstractTest.java |  283 +
 .../IgniteClientDataStructuresTest.java         |   28 +
 ...IgniteClientDiscoveryDataStructuresTest.java |   28 +
 ...CacheLoadingConcurrentGridStartSelfTest.java |  163 +
 .../CacheNoValueClassOnServerNodeTest.java      |  129 +
 .../GridCacheAbstractNodeRestartSelfTest.java   |  101 +-
 .../GridCacheAtomicTimeoutSelfTest.java         |  314 -
 .../GridCacheClientModesAbstractSelfTest.java   |   94 +-
 ...ientModesTcpClientDiscoveryAbstractTest.java |  168 +
 ...GridCacheLoadingConcurrentGridStartTest.java |  154 -
 .../distributed/GridCacheLockAbstractTest.java  |   73 +
 .../distributed/GridCacheMixedModeSelfTest.java |    3 +
 ...ridCachePartitionNotLoadedEventSelfTest.java |   22 +-
 .../IgniteCacheAtomicMessageRecoveryTest.java   |   32 +
 ...niteCacheClientNodeChangingTopologyTest.java | 1803 +++++
 .../IgniteCacheClientNodeConcurrentStart.java   |  105 +
 ...teCacheClientNodePartitionsExchangeTest.java |  632 ++
 .../IgniteCacheMessageRecoveryAbstractTest.java |  175 +
 .../IgniteCachePutGetRestartAbstractTest.java   |  234 +
 .../IgniteCacheTxFairAffinityNodeJoinTest.java  |   35 +
 .../IgniteCacheTxMessageRecoveryTest.java       |   32 +
 ...arDisabledFairAffinityPutGetRestartTest.java |   35 +
 ...iteCacheTxNearDisabledPutGetRestartTest.java |   30 +
 .../IgniteCrossCacheTxStoreSelfTest.java        |  147 +-
 .../distributed/IgniteTxGetAfterStopTest.java   |  131 +
 ...xOriginatingNodeFailureAbstractSelfTest.java |    8 +-
 ...cOriginatingNodeFailureAbstractSelfTest.java |    7 +-
 .../dht/GridCacheClientOnlySelfTest.java        |   60 +-
 .../GridCacheDhtClientRemoveFailureTest.java    |   28 +
 ...GridCacheDhtEvictionNearReadersSelfTest.java |   11 +-
 .../dht/GridCacheDhtEvictionSelfTest.java       |   11 +-
 .../dht/GridCacheDhtPreloadSelfTest.java        |    6 +-
 ...icOffHeapTieredMultiNodeFullApiSelfTest.java |   43 +
 ...ledFairAffinityMultiNodeFullApiSelfTest.java |   36 +
 ...achePartitionedNearDisabledLockSelfTest.java |   47 +
 ...ionedNearDisabledOffHeapFullApiSelfTest.java |    8 +-
 ...DisabledOffHeapMultiNodeFullApiSelfTest.java |    8 +-
 ...abledOffHeapTieredAtomicFullApiSelfTest.java |   56 +
 ...earDisabledOffHeapTieredFullApiSelfTest.java |   33 +
 ...edOffHeapTieredMultiNodeFullApiSelfTest.java |   33 +
 ...itionedTxOriginatingNodeFailureSelfTest.java |    2 -
 .../dht/IgniteCacheMultiTxLockSelfTest.java     |   55 +-
 ...rDisabledPrimaryNodeFailureRecoveryTest.java |   31 +
 ...rtitionedPrimaryNodeFailureRecoveryTest.java |   31 +
 ...woBackupsPrimaryNodeFailureRecoveryTest.java |   37 +
 ...ePrimaryNodeFailureRecoveryAbstractTest.java |  533 ++
 ...cClientInvalidPartitionHandlingSelfTest.java |   29 +
 .../GridCacheAtomicClientRemoveFailureTest.java |   28 +
 ...eAtomicInvalidPartitionHandlingSelfTest.java |   37 +-
 ...idCacheAtomicReplicatedFailoverSelfTest.java |    6 +
 ...unctionExcludeNeighborsAbstractSelfTest.java |    3 +-
 ...nlyFairAffinityMultiNodeFullApiSelfTest.java |   36 +
 ...micFairAffinityMultiNodeFullApiSelfTest.java |   35 +
 ...ledFairAffinityMultiNodeFullApiSelfTest.java |   36 +
 .../near/GridCacheAtomicNearOnlySelfTest.java   |   32 -
 ...CacheAtomicOffHeapTieredFullApiSelfTest.java |   32 +
 ...icOffHeapTieredMultiNodeFullApiSelfTest.java |   33 +
 ...derFairAffinityMultiNodeFullApiSelfTest.java |   36 +
 ...yWriteOrderOffHeapTieredFullApiSelfTest.java |   33 +
 ...erOffHeapTieredMultiNodeFullApiSelfTest.java |   33 +
 ...nlyFairAffinityMultiNodeFullApiSelfTest.java |   35 +
 ...idCacheNearOnlyMultiNodeFullApiSelfTest.java |    3 +
 .../near/GridCacheNearOnlySelfTest.java         |   63 +-
 .../near/GridCacheNearOnlyTopologySelfTest.java |    1 +
 ...ionedClientOnlyNoPrimaryFullApiSelfTest.java |    5 +-
 .../GridCachePartitionedEvictionSelfTest.java   |   11 +-
 .../GridCachePartitionedFullApiSelfTest.java    |   32 +
 ...achePartitionedMultiNodeFullApiSelfTest.java |   15 +-
 ...ePartitionedMultiThreadedPutGetSelfTest.java |    6 +-
 .../GridCachePartitionedNodeRestartTest.java    |    4 +-
 ...dCachePartitionedOffHeapFullApiSelfTest.java |    8 +-
 ...titionedOffHeapMultiNodeFullApiSelfTest.java |    8 +-
 ...PartitionedOffHeapTieredFullApiSelfTest.java |   32 +
 ...edOffHeapTieredMultiNodeFullApiSelfTest.java |   72 +
 ...ePartitionedOptimisticTxNodeRestartTest.java |    4 +-
 ...achePartitionedPreloadLifecycleSelfTest.java |    2 +-
 .../GridCachePartitionedTxSalvageSelfTest.java  |   25 +-
 ...idCacheRendezvousAffinityClientSelfTest.java |    4 +
 .../near/IgniteCacheNearOnlyTxTest.java         |  190 +
 .../near/IgniteCacheNearTxRollbackTest.java     |  133 +
 .../near/NoneRebalanceModeSelfTest.java         |   67 +
 ...nedFairAffinityMultiNodeFullApiSelfTest.java |   37 +
 .../GridCacheReplicatedClientOnlySelfTest.java  |   43 -
 .../GridCacheReplicatedFailoverSelfTest.java    |    6 +
 .../GridCacheReplicatedLockSelfTest.java        |    5 +
 .../GridCacheReplicatedNearOnlySelfTest.java    |   43 -
 .../GridCacheReplicatedNodeRestartSelfTest.java |   82 +
 ...idCacheReplicatedOffHeapFullApiSelfTest.java |    8 +-
 ...plicatedOffHeapMultiNodeFullApiSelfTest.java |    8 +-
 ...eReplicatedOffHeapTieredFullApiSelfTest.java |   33 +
 ...edOffHeapTieredMultiNodeFullApiSelfTest.java |   33 +
 ...dezvousAffinityMultiNodeFullApiSelfTest.java |   35 -
 .../GridCacheSyncReplicatedPreloadSelfTest.java |    1 -
 ...CacheReplicatedPreloadLifecycleSelfTest.java |    2 +-
 .../cache/eviction/EvictionAbstractTest.java    | 1056 +++
 .../GridCacheBatchEvictUnswapSelfTest.java      |    5 +-
 ...heConcurrentEvictionConsistencySelfTest.java |   82 +-
 .../GridCacheConcurrentEvictionsSelfTest.java   |   29 +-
 .../GridCacheDistributedEvictionsSelfTest.java  |    5 +-
 .../GridCacheEmptyEntriesAbstractSelfTest.java  |   11 +-
 .../eviction/GridCacheEvictionAbstractTest.java |  484 --
 .../GridCacheEvictionTouchSelfTest.java         |   22 +-
 .../cache/eviction/GridCacheMockEntry.java      |    5 +
 .../fifo/FifoEvictionPolicySelfTest.java        |  262 +
 ...ridCacheFifoBatchEvictionPolicySelfTest.java |  384 -
 .../GridCacheFifoEvictionPolicySelfTest.java    |  372 -
 .../lru/GridCacheLruEvictionPolicySelfTest.java |  417 -
 .../GridCacheLruNearEvictionPolicySelfTest.java |  136 -
 ...heNearOnlyLruNearEvictionPolicySelfTest.java |  171 -
 .../eviction/lru/LruEvictionPolicySelfTest.java |  353 +
 .../lru/LruNearEvictionPolicySelfTest.java      |  140 +
 .../LruNearOnlyNearEvictionPolicySelfTest.java  |  172 +
 .../GridCacheRandomEvictionPolicySelfTest.java  |  258 -
 .../RandomEvictionPolicyCacheSizeSelfTest.java  |   78 +
 .../random/RandomEvictionPolicySelfTest.java    |  357 +
 .../SortedEvictionPolicyPerformanceTest.java    |  134 +
 .../sorted/SortedEvictionPolicySelfTest.java    |  266 +
 .../IgniteCacheExpiryPolicyAbstractTest.java    |    2 +-
 .../IgniteCacheExpiryPolicyTestSuite.java       |    2 +
 .../expiry/IgniteCacheTtlCleanupSelfTest.java   |   85 +
 ...CacheLocalOffHeapAndSwapMetricsSelfTest.java |  412 +
 ...LocalAtomicOffHeapTieredFullApiSelfTest.java |   32 +
 .../GridCacheLocalIsolatedNodesSelfTest.java    |  118 +
 .../GridCacheLocalOffHeapFullApiSelfTest.java   |    6 +-
 ...dCacheLocalOffHeapTieredFullApiSelfTest.java |   32 +
 .../GridCacheSwapScanQueryAbstractSelfTest.java |   26 +-
 ...ridCacheContinuousQueryAbstractSelfTest.java |    6 +-
 .../continuous/GridEventConsumeSelfTest.java    |   93 +-
 .../DataStreamProcessorSelfTest.java            |    1 +
 .../DataStreamerMultiThreadedSelfTest.java      |  110 +
 .../igfs/IgfsClientCacheSelfTest.java           |  133 +
 .../processors/igfs/IgfsCommonAbstractTest.java |   10 -
 .../processors/igfs/IgfsOneClientNodeTest.java  |  131 +
 .../processors/igfs/IgfsSizeSelfTest.java       |    2 +-
 .../processors/igfs/IgfsStreamsSelfTest.java    |    4 +-
 .../cache/GridCacheCommandHandlerSelfTest.java  |   10 +-
 .../service/ClosureServiceClientsNodesTest.java |  251 +
 .../service/GridServiceClientNodeTest.java      |   81 +
 .../GridServiceReassignmentSelfTest.java        |    2 +-
 .../util/nio/GridNioDelimitedBufferTest.java    |  112 +
 ...idFileSwapSpaceSpiMultithreadedLoadTest.java |    4 +-
 .../loadtests/GridCacheMultiNodeLoadTest.java   |    5 +-
 .../GridContinuousOperationsLoadTest.java       |    3 +-
 .../GridCachePartitionedAtomicLongLoadTest.java |    6 +-
 .../loadtests/hashmap/GridCacheTestContext.java |    3 +-
 .../swap/GridSwapEvictAllBenchmark.java         |    6 +-
 .../logger/java/IgniteJavaLoggerTest.java       |   65 -
 .../ignite/logger/java/JavaLoggerTest.java      |   65 +
 .../OptimizedMarshallerNodeFailoverTest.java    |    4 +-
 ...GridMessagingNoPeerClassLoadingSelfTest.java |    7 +-
 .../ignite/messaging/GridMessagingSelfTest.java |   13 +-
 .../GridTcpCommunicationSpiConfigSelfTest.java  |    1 -
 .../discovery/AbstractDiscoverySelfTest.java    |    8 +-
 ...pClientDiscoveryMarshallerCheckSelfTest.java |   76 +
 .../tcp/TcpClientDiscoverySelfTest.java         |  692 --
 .../tcp/TcpClientDiscoverySpiMulticastTest.java |  129 +
 .../tcp/TcpClientDiscoverySpiSelfTest.java      | 1171 +++
 .../tcp/TcpDiscoveryConcurrentStartTest.java    |   61 +-
 .../tcp/TcpDiscoveryMultiThreadedTest.java      |   18 +-
 .../discovery/tcp/TcpDiscoveryRestartTest.java  |  199 +
 .../spi/discovery/tcp/TcpDiscoverySelfTest.java |   72 +-
 .../GridSwapSpaceSpiAbstractSelfTest.java       |    2 +-
 .../stream/socket/SocketStreamerSelfTest.java   |  315 +
 .../ignite/stream/socket/package-info.java      |   21 +
 .../testframework/GridSpiTestContext.java       |   30 +-
 .../ignite/testframework/GridTestUtils.java     |   15 +
 .../testframework/junits/GridAbstractTest.java  |   67 +-
 .../junits/cache/TestCacheSession.java          |   18 +
 .../cache/TestThreadLocalCacheSession.java      |   15 +
 .../junits/common/GridCommonAbstractTest.java   |  119 +-
 .../junits/logger/GridTestLog4jLogger.java      |   10 +-
 .../ignite/testsuites/IgniteBasicTestSuite.java |    7 +
 .../IgniteCacheDataStructuresSelfTestSuite.java |    3 +
 .../IgniteCacheEvictionSelfTestSuite.java       |   14 +-
 .../IgniteCacheFailoverTestSuite.java           |   18 +-
 .../IgniteCacheFullApiSelfTestSuite.java        |   27 +-
 .../IgniteCacheMetricsSelfTestSuite.java        |    1 +
 .../IgniteCacheNearOnlySelfTestSuite.java       |   16 +-
 ...niteCacheP2pUnmarshallingErrorTestSuite.java |   41 +
 .../testsuites/IgniteCacheRestartTestSuite.java |   16 +-
 .../IgniteCacheTcpClientDiscoveryTestSuite.java |   47 +
 .../ignite/testsuites/IgniteCacheTestSuite.java |  296 +-
 .../testsuites/IgniteCacheTestSuite2.java       |  143 +
 .../testsuites/IgniteCacheTestSuite3.java       |  142 +
 .../testsuites/IgniteCacheTestSuite4.java       |  140 +
 .../IgniteCacheTxRecoverySelfTestSuite.java     |    4 +
 .../ignite/testsuites/IgniteIgfsTestSuite.java  |    3 +
 .../testsuites/IgniteKernalSelfTestSuite.java   |    7 +-
 .../testsuites/IgniteLoggingSelfTestSuite.java  |    2 +-
 .../IgniteSpiDiscoverySelfTestSuite.java        |    4 +-
 .../testsuites/IgniteStreamTestSuite.java       |   39 +
 .../IgniteTopologyValidatorTestSuit.java        |   43 +
 .../testsuites/IgniteUtilSelfTestSuite.java     |    1 +
 .../ignite/util/TestTcpCommunicationSpi.java    |   54 +
 modules/extdata/p2p/pom.xml                     |    2 +-
 .../CacheNoValueClassOnServerTestClient.java    |   88 +
 .../apache/ignite/tests/p2p/cache/Person.java   |   42 +
 .../CacheConfigurationP2PTestClient.java        |    1 -
 modules/extdata/uri/pom.xml                     |    2 +-
 modules/gce/README.txt                          |   32 +
 modules/gce/licenses/apache-2.0.txt             |  202 +
 modules/gce/pom.xml                             |   92 +
 .../gce/TcpDiscoveryGoogleStorageIpFinder.java  |  383 +
 .../tcp/ipfinder/gce/package-info.java          |   22 +
 ...pDiscoveryGoogleStorageIpFinderSelfTest.java |   73 +
 .../tcp/ipfinder/gce/package-info.java          |   22 +
 .../ignite/testsuites/IgniteGCETestSuite.java   |   71 +
 modules/geospatial/pom.xml                      |    2 +-
 .../query/h2/GridH2IndexingGeoSelfTest.java     |   72 +-
 modules/hadoop/pom.xml                          |    2 +-
 .../fs/IgniteHadoopIgfsSecondaryFileSystem.java |  165 +-
 .../hadoop/fs/v1/IgniteHadoopFileSystem.java    |  107 +-
 .../hadoop/fs/v2/IgniteHadoopFileSystem.java    |   32 +-
 .../processors/hadoop/HadoopDefaultJobInfo.java |    3 +
 .../processors/hadoop/HadoopProcessor.java      |    2 +-
 .../internal/processors/hadoop/HadoopUtils.java |   10 +-
 .../hadoop/SecondaryFileSystemProvider.java     |   53 +-
 .../hadoop/fs/HadoopDistributedFileSystem.java  |   91 -
 .../hadoop/fs/HadoopFileSystemsUtils.java       |   17 -
 .../hadoop/fs/HadoopLazyConcurrentMap.java      |  204 +
 .../processors/hadoop/igfs/HadoopIgfsEx.java    |    6 +
 .../hadoop/igfs/HadoopIgfsInProc.java           |  170 +-
 .../processors/hadoop/igfs/HadoopIgfsIpcIo.java |    5 +-
 .../hadoop/igfs/HadoopIgfsOutProc.java          |   33 +-
 .../hadoop/igfs/HadoopIgfsWrapper.java          |   19 +-
 .../hadoop/jobtracker/HadoopJobTracker.java     |   19 +-
 .../hadoop/taskexecutor/HadoopRunnableTask.java |    3 +
 .../external/HadoopExternalTaskExecutor.java    |    3 +
 .../processors/hadoop/v2/HadoopV2Context.java   |   10 +-
 .../processors/hadoop/v2/HadoopV2Job.java       |   11 +-
 .../hadoop/v2/HadoopV2TaskContext.java          |   13 +-
 .../HadoopIgfs20FileSystemAbstractSelfTest.java |   56 +-
 ...oopSecondaryFileSystemConfigurationTest.java |    4 +-
 .../igfs/IgfsNearOnlyMultiNodeSelfTest.java     |    5 +-
 .../IgniteHadoopFileSystemAbstractSelfTest.java |   63 +-
 .../IgniteHadoopFileSystemClientSelfTest.java   |    2 +-
 .../IgniteHadoopFileSystemIpcCacheSelfTest.java |    2 +
 .../HadoopDefaultMapReducePlannerSelfTest.java  |    8 +-
 .../hadoop/HadoopFileSystemsTest.java           |   23 +-
 .../collections/HadoopSkipListSelfTest.java     |    4 +-
 .../testsuites/IgniteHadoopTestSuite.java       |    2 +-
 modules/hibernate/pom.xml                       |    2 +-
 .../HibernateAccessStrategyAdapter.java         |   10 +-
 .../hibernate/HibernateCollectionRegion.java    |    2 +-
 .../cache/hibernate/HibernateEntityRegion.java  |    2 +-
 .../hibernate/HibernateGeneralDataRegion.java   |    2 +-
 .../hibernate/HibernateNaturalIdRegion.java     |    2 +-
 .../HibernateNonStrictAccessStrategy.java       |    4 +-
 .../hibernate/HibernateQueryResultsRegion.java  |    2 +-
 .../HibernateReadOnlyAccessStrategy.java        |    2 +-
 .../HibernateReadWriteAccessStrategy.java       |    2 +-
 .../ignite/cache/hibernate/HibernateRegion.java |    4 +-
 .../cache/hibernate/HibernateRegionFactory.java |   10 +-
 .../hibernate/HibernateTimestampsRegion.java    |    2 +-
 .../HibernateTransactionalAccessStrategy.java   |    4 +-
 .../HibernateTransactionalDataRegion.java       |    2 +-
 .../CacheHibernateStoreSessionListener.java     |  216 +
 .../HibernateL2CacheConfigurationSelfTest.java  |    2 +-
 .../hibernate/HibernateL2CacheSelfTest.java     |    2 +-
 ...heHibernateStoreSessionListenerSelfTest.java |  228 +
 .../testsuites/IgniteHibernateTestSuite.java    |    2 +
 modules/indexing/pom.xml                        |    2 +-
 .../processors/query/h2/IgniteH2Indexing.java   |  203 +-
 .../query/h2/opt/GridH2AbstractKeyValueRow.java |  110 +-
 .../query/h2/opt/GridH2KeyValueRowOffheap.java  |   11 +-
 .../query/h2/opt/GridH2KeyValueRowOnheap.java   |    6 +-
 .../query/h2/opt/GridH2RowDescriptor.java       |   14 +-
 .../processors/query/h2/opt/GridH2Table.java    |   10 +-
 .../query/h2/opt/GridH2ValueCacheObject.java    |  191 +
 .../query/h2/opt/GridLuceneIndex.java           |   84 +-
 .../query/h2/sql/GridSqlFunction.java           |    6 +-
 .../query/h2/sql/GridSqlOperationType.java      |    2 +-
 .../query/h2/sql/GridSqlPlaceholder.java        |   51 +
 .../processors/query/h2/sql/GridSqlQuery.java   |   20 +
 .../query/h2/sql/GridSqlQueryParser.java        |   16 +-
 .../query/h2/sql/GridSqlQuerySplitter.java      |   15 +-
 .../processors/query/h2/sql/GridSqlSelect.java  |    2 +-
 .../processors/query/h2/sql/GridSqlUnion.java   |    2 +-
 .../query/h2/twostep/GridMapQueryExecutor.java  |   61 +-
 .../query/h2/twostep/GridMergeIndex.java        |    6 +-
 .../h2/twostep/GridMergeIndexUnsorted.java      |    4 +-
 .../h2/twostep/GridReduceQueryExecutor.java     |  164 +-
 .../query/h2/twostep/GridResultPage.java        |   80 +-
 .../query/h2/twostep/msg/GridH2Array.java       |  124 +
 .../query/h2/twostep/msg/GridH2Boolean.java     |  112 +
 .../query/h2/twostep/msg/GridH2Byte.java        |  113 +
 .../query/h2/twostep/msg/GridH2Bytes.java       |  113 +
 .../query/h2/twostep/msg/GridH2CacheObject.java |  148 +
 .../query/h2/twostep/msg/GridH2Date.java        |  115 +
 .../query/h2/twostep/msg/GridH2Decimal.java     |  134 +
 .../query/h2/twostep/msg/GridH2Double.java      |  113 +
 .../query/h2/twostep/msg/GridH2Float.java       |  113 +
 .../query/h2/twostep/msg/GridH2Geometry.java    |  134 +
 .../query/h2/twostep/msg/GridH2Integer.java     |  113 +
 .../query/h2/twostep/msg/GridH2JavaObject.java  |  113 +
 .../query/h2/twostep/msg/GridH2Long.java        |  113 +
 .../query/h2/twostep/msg/GridH2Null.java        |   78 +
 .../query/h2/twostep/msg/GridH2Short.java       |  113 +
 .../query/h2/twostep/msg/GridH2String.java      |  115 +
 .../query/h2/twostep/msg/GridH2Time.java        |  116 +
 .../query/h2/twostep/msg/GridH2Timestamp.java   |  133 +
 .../query/h2/twostep/msg/GridH2Uuid.java        |  133 +
 .../h2/twostep/msg/GridH2ValueMessage.java      |   49 +
 .../twostep/msg/GridH2ValueMessageFactory.java  |  201 +
 .../GridCacheAbstractFieldsQuerySelfTest.java   | 1282 ----
 .../cache/GridCacheCrossCacheQuerySelfTest.java |   68 +-
 .../cache/GridCacheOffHeapAndSwapSelfTest.java  |   11 +-
 .../cache/GridCacheOffHeapSelfTest.java         |   11 +-
 .../GridCacheOffheapIndexEntryEvictTest.java    |  200 +
 .../cache/GridCacheOffheapIndexGetSelfTest.java |  123 +
 .../cache/GridCacheQueryMetricsSelfTest.java    |  106 +-
 .../GridCacheQuerySerializationSelfTest.java    |  144 +
 ...idCacheReduceQueryMultithreadedSelfTest.java |   10 +-
 .../cache/GridIndexingWithNoopSwapSelfTest.java |   23 +-
 .../IgniteCacheAbstractFieldsQuerySelfTest.java |  256 +-
 .../cache/IgniteCacheAbstractQuerySelfTest.java |    6 +-
 ...acheConfigurationPrimitiveTypesSelfTest.java |  104 +
 ...niteCacheP2pUnmarshallingQueryErrorTest.java |   57 +
 ...QueryMultiThreadedOffHeapTieredSelfTest.java |   37 +
 .../IgniteCacheQueryMultiThreadedSelfTest.java  |   52 +-
 ...artitionedFieldsQueryP2PEnabledSelfTest.java |   34 -
 ...GridCachePartitionedFieldsQuerySelfTest.java |  115 -
 ...rtitionedFieldsQueryP2PDisabledSelfTest.java |   34 -
 ...artitionedFieldsQueryP2PEnabledSelfTest.java |   34 +
 ...eReplicatedFieldsQueryP2PEnableSelfTest.java |   34 -
 .../GridCacheReplicatedFieldsQuerySelfTest.java |  157 -
 ...eplicatedFieldsQueryP2PDisabledSelfTest.java |   34 -
 ...ReplicatedFieldsQueryP2PEnabledSelfTest.java |   34 +
 .../local/IgniteCacheLocalQuerySelfTest.java    |    6 +
 ...dCacheAbstractReduceFieldsQuerySelfTest.java |   11 +-
 ...cheReduceFieldsQueryPartitionedSelfTest.java |    5 +-
 .../cache/ttl/CacheTtlAbstractSelfTest.java     |    6 +-
 .../query/h2/GridH2IndexRebuildTest.java        |   18 +-
 .../h2/GridIndexingSpiAbstractSelfTest.java     |  132 +-
 .../h2/sql/AbstractH2CompareQueryTest.java      |   49 +-
 .../query/h2/sql/BaseH2CompareQueryTest.java    |   48 +-
 .../query/h2/sql/GridQueryParsingTest.java      |    9 +
 .../query/h2/sql/H2CompareBigQueryTest.java     |    2 +-
 .../IgniteCacheQuerySelfTestSuite.java          |   14 +-
 .../IgniteCacheWithIndexingTestSuite.java       |    5 +
 .../processors/query/h2/sql/bigQuery.sql        |    2 +-
 modules/jcl/pom.xml                             |    2 +-
 .../ignite/logger/jcl/IgniteJclLogger.java      |  167 -
 .../org/apache/ignite/logger/jcl/JclLogger.java |  167 +
 .../ignite/logger/jcl/IgniteJclLoggerTest.java  |   48 -
 .../apache/ignite/logger/jcl/JclLoggerTest.java |   48 +
 .../ignite/testsuites/IgniteJclTestSuite.java   |    2 +-
 modules/jta/pom.xml                             |    2 +-
 .../processors/cache/jta/CacheJtaManager.java   |    4 +-
 modules/log4j/pom.xml                           |    2 +-
 .../apache/ignite/logger/log4j/Log4JLogger.java |    8 +-
 modules/mesos/README.txt                        |   28 +
 modules/mesos/licenses/apache-2.0.txt           |  202 +
 modules/mesos/licenses/jetty-epl-license.txt    |   69 +
 modules/mesos/pom.xml                           |  101 +
 .../apache/ignite/mesos/ClusterProperties.java  |  519 ++
 .../apache/ignite/mesos/IgniteFramework.java    |  119 +
 .../apache/ignite/mesos/IgniteScheduler.java    |  361 +
 .../org/apache/ignite/mesos/IgniteTask.java     |   86 +
 .../org/apache/ignite/mesos/package-info.java   |   22 +
 .../ignite/mesos/resource/IgniteProvider.java   |  234 +
 .../ignite/mesos/resource/JettyServer.java      |   61 +
 .../ignite/mesos/resource/ResourceHandler.java  |  142 +
 .../ignite/mesos/resource/ResourceProvider.java |  120 +
 .../ignite/mesos/resource/package-info.java     |   22 +
 .../main/resources/ignite-default-config.xml    |   35 +
 .../org/apache/ignite/IgniteMesosTestSuite.java |   38 +
 .../ignite/mesos/IgniteSchedulerSelfTest.java   |  464 ++
 modules/rest-http/pom.xml                       |    2 +-
 .../http/jetty/GridJettyRestHandler.java        |    3 +
 modules/scalar/pom.xml                          |    2 +-
 .../ignite/scalar/ScalarConversions.scala       |    8 -
 .../scalar/tests/ScalarCacheQueriesSpec.scala   |  154 +-
 .../ignite/scalar/tests/ScalarCacheSpec.scala   |   23 +-
 .../scalar/tests/ScalarConversionsSpec.scala    |   43 +-
 .../scalar/tests/ScalarProjectionSpec.scala     |  128 +-
 .../scalar/tests/ScalarReturnableSpec.scala     |   41 +-
 modules/schedule/pom.xml                        |    2 +-
 modules/schema-import/pom.xml                   |    8 +-
 .../ignite/schema/generator/CodeGenerator.java  |  105 +-
 .../ignite/schema/ui/SchemaImportApp.java       |    8 +-
 modules/slf4j/pom.xml                           |    2 +-
 .../ignite/logger/slf4j/GridSlf4jLogger.java    |  138 -
 .../apache/ignite/logger/slf4j/Slf4jLogger.java |  138 +
 modules/spring/pom.xml                          |   16 +-
 .../spring/CacheSpringStoreSessionListener.java |  207 +
 .../util/spring/IgniteSpringHelperImpl.java     |  217 +-
 ...CacheSpringStoreSessionListenerSelfTest.java |  197 +
 .../ignite/internal/GridFactorySelfTest.java    |    3 +-
 .../IgniteStartFromStreamConfigurationTest.java |   50 +
 .../testsuites/IgniteSpringTestSuite.java       |    5 +
 modules/ssh/pom.xml                             |    2 +-
 ...gniteProjectionStartStopRestartSelfTest.java |   26 +-
 modules/tools/pom.xml                           |    2 +-
 modules/urideploy/pom.xml                       |    2 +-
 .../uri/GridUriDeploymentClassLoader.java       |    4 +-
 .../spi/deployment/uri/UriDeploymentSpi.java    |    2 +-
 .../licenses/jcraft-revised-bsd.txt             |   28 -
 modules/visor-console/pom.xml                   |    8 +-
 .../ignite/visor/commands/VisorConsole.scala    |  316 +-
 .../visor/commands/VisorConsoleCommand.scala    |   77 -
 .../ignite/visor/commands/VisorTextTable.scala  |  539 --
 .../visor/commands/ack/VisorAckCommand.scala    |   42 +-
 .../commands/alert/VisorAlertCommand.scala      |   35 +-
 .../commands/cache/VisorCacheClearCommand.scala |   51 +-
 .../commands/cache/VisorCacheCommand.scala      |   36 +-
 .../commands/cache/VisorCacheScanCommand.scala  |   60 +-
 .../commands/cache/VisorCacheStopCommand.scala  |   30 +-
 .../commands/cache/VisorCacheSwapCommand.scala  |   66 +-
 .../commands/common/VisorConsoleCommand.scala   |   90 +
 .../visor/commands/common/VisorTextTable.scala  |  543 ++
 .../config/VisorConfigurationCommand.scala      |  438 +-
 .../commands/deploy/VisorDeployCommand.scala    |   47 +-
 .../commands/disco/VisorDiscoveryCommand.scala  |   58 +-
 .../commands/events/VisorEventsCommand.scala    |  338 +-
 .../visor/commands/gc/VisorGcCommand.scala      |  130 +-
 .../visor/commands/kill/VisorKillCommand.scala  |   53 +-
 .../visor/commands/node/VisorNodeCommand.scala  |   49 +-
 .../visor/commands/ping/VisorPingCommand.scala  |   41 +-
 .../commands/start/VisorStartCommand.scala      |   34 +-
 .../commands/tasks/VisorTasksCommand.scala      |   78 +-
 .../commands/top/VisorTopologyCommand.scala     |   36 +-
 .../visor/commands/vvm/VisorVvmCommand.scala    |   32 +-
 .../scala/org/apache/ignite/visor/visor.scala   |  349 +-
 .../ignite/visor/VisorRuntimeBaseSpec.scala     |    2 +-
 .../ignite/visor/VisorTextTableSpec.scala       |    3 +-
 .../visor/commands/VisorArgListSpec.scala       |   60 +-
 .../commands/VisorFileNameCompleterSpec.scala   |   34 +-
 .../commands/ack/VisorAckCommandSpec.scala      |   20 +-
 .../commands/alert/VisorAlertCommandSpec.scala  |   68 +-
 .../cache/VisorCacheClearCommandSpec.scala      |   48 +-
 .../commands/cache/VisorCacheCommandSpec.scala  |   66 +-
 .../config/VisorConfigurationCommandSpec.scala  |    8 +-
 .../cswap/VisorCacheSwapCommandSpec.scala       |   24 +-
 .../deploy/VisorDeployCommandSpec.scala         |   10 +-
 .../disco/VisorDiscoveryCommandSpec.scala       |   46 +-
 .../events/VisorEventsCommandSpec.scala         |   28 +-
 .../visor/commands/gc/VisorGcCommandSpec.scala  |   30 +-
 .../commands/help/VisorHelpCommandSpec.scala    |   57 +-
 .../commands/kill/VisorKillCommandSpec.scala    |   58 +-
 .../commands/log/VisorLogCommandSpec.scala      |   10 +-
 .../commands/mem/VisorMemoryCommandSpec.scala   |   77 +-
 .../commands/node/VisorNodeCommandSpec.scala    |   22 +-
 .../commands/open/VisorOpenCommandSpec.scala    |   16 +-
 .../commands/ping/VisorPingCommandSpec.scala    |   16 +-
 .../commands/start/VisorStartCommandSpec.scala  |  126 +-
 .../commands/tasks/VisorTasksCommandSpec.scala  |  112 +-
 .../commands/top/VisorTopologyCommandSpec.scala |   52 +-
 .../commands/vvm/VisorVvmCommandSpec.scala      |   30 +-
 modules/visor-plugins/pom.xml                   |    2 +-
 modules/web/pom.xml                             |    2 +-
 .../config/benchmark-client-mode.properties     |   89 +
 modules/yardstick/pom.xml                       |    2 +-
 .../cache/IgniteCacheAbstractBenchmark.java     |    2 +-
 .../jdbc/IgniteJdbcStoreAbstractBenchmark.java  |    4 +-
 parent/pom.xml                                  |    9 +
 pom.xml                                         |  362 +-
 scripts/git-apply-patch.sh                      |    8 +-
 scripts/git-format-patch.sh                     |   16 +-
 scripts/git-patch-functions.sh                  |   36 +-
 scripts/git-patch-prop.sh                       |    2 +-
 1021 files changed, 65871 insertions(+), 37944 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1f516a25/modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCacheMetrics.java
----------------------------------------------------------------------
diff --cc modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCacheMetrics.java
index 65a7f1b,30be424..c5d70a5
--- a/modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCacheMetrics.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCacheMetrics.java
@@@ -212,14 -202,13 +205,13 @@@ public class VisorCacheMetrics implemen
          cm.avgPutTime = m.getAveragePutTime();
          cm.avgRemovalTime = m.getAverageRemoveTime();
  
-         cm.readsPerSec = (int)(MICROSECONDS_IN_SECOND * 1.f / m.getAverageGetTime());
-         cm.writesPerSec = (int)(MICROSECONDS_IN_SECOND * 1.f / m.getAveragePutTime());
-         cm.hitsPerSec = -1;
-         cm.missesPerSec = (int)(MICROSECONDS_IN_SECOND * 1.f / m.getAverageRemoveTime());
-         cm.commitsPerSec = (int)(MICROSECONDS_IN_SECOND * 1.f / m.getAverageTxCommitTime());
-         cm.rollbacksPerSec = (int)(MICROSECONDS_IN_SECOND * 1.f / m.getAverageTxRollbackTime());
+         cm.readsPerSec = perSecond(m.getAverageGetTime());
+         cm.putsPerSec = perSecond(m.getAveragePutTime());
+         cm.removalsPerSec = perSecond(m.getAverageRemoveTime());
+         cm.commitsPerSec = perSecond(m.getAverageTxCommitTime());
+         cm.rollbacksPerSec = perSecond(m.getAverageTxRollbackTime());
  
 -        cm.qryMetrics = VisorCacheQueryMetrics.from(c.context().queries().metrics());
 +        cm.qryMetrics = VisorCacheQueryMetrics.from(c.queryMetrics());
  
          cm.dhtEvictQueueCurrSize = m.getDhtEvictQueueCurrentSize();
          cm.txThreadMapSize = m.getTxThreadMapSize();

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1f516a25/modules/core/src/main/java/org/apache/ignite/internal/visor/cache/VisorCacheStoreConfiguration.java
----------------------------------------------------------------------


[26/42] incubator-ignite git commit: Merge remote-tracking branch 'remotes/origin/ignite-sprint-5' into ignite-883_1

Posted by ag...@apache.org.
Merge remote-tracking branch 'remotes/origin/ignite-sprint-5' into ignite-883_1


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

Branch: refs/heads/ignite-389
Commit: 75006b805a9997354ea9a1c645ad11ab8f9ba631
Parents: e237b00 5f06f57
Author: sboikov <sb...@gridgain.com>
Authored: Fri Jun 5 18:11:06 2015 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Fri Jun 5 18:11:06 2015 +0300

----------------------------------------------------------------------
 .../affinity/GridAffinityProcessor.java         | 23 ++++-
 .../datastreamer/DataStreamerImpl.java          | 92 ++++++++++++++------
 .../DataStreamerMultiThreadedSelfTest.java      | 59 +++++++++----
 .../IgniteMessagingWithClientTest.java          |  2 +
 4 files changed, 131 insertions(+), 45 deletions(-)
----------------------------------------------------------------------



[30/42] incubator-ignite git commit: Merge remote-tracking branch 'remotes/origin/ignite-sprint-5' into ignite-883_1

Posted by ag...@apache.org.
Merge remote-tracking branch 'remotes/origin/ignite-sprint-5' into ignite-883_1


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

Branch: refs/heads/ignite-389
Commit: 0a4e7dd4c8321bb163cd8568494c97a1a4aef437
Parents: 50b0b49 59db4a5
Author: sboikov <sb...@gridgain.com>
Authored: Mon Jun 8 10:10:59 2015 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Mon Jun 8 10:10:59 2015 +0300

----------------------------------------------------------------------
 .../java/org/apache/ignite/IgniteCache.java     |  11 +
 .../apache/ignite/internal/IgniteKernal.java    |   2 +
 .../ignite/internal/IgniteNodeAttributes.java   |   5 +-
 .../cache/DynamicCacheDescriptor.java           |  17 +
 .../processors/cache/GridCacheAdapter.java      |  17 +-
 .../processors/cache/GridCacheContext.java      |  13 +
 .../processors/cache/GridCacheProcessor.java    |  37 ++-
 .../processors/cache/GridCacheProxyImpl.java    |  14 +-
 .../processors/cache/IgniteCacheProxy.java      |  23 ++
 .../processors/cache/IgniteInternalCache.java   |  11 +-
 .../dht/atomic/GridDhtAtomicCache.java          |   4 +
 .../transactions/IgniteTxLocalAdapter.java      |  28 ++
 .../processors/igfs/IgfsMetaManager.java        |   2 +-
 .../IgniteTxRollbackCheckedException.java       |   9 +
 ...acheReadOnlyTransactionalClientSelfTest.java | 327 +++++++++++++++++++
 .../cache/GridCacheAbstractFullApiSelfTest.java |  83 +++++
 .../GridCacheExAbstractFullApiSelfTest.java     | 103 ------
 .../GridCacheExColocatedFullApiSelfTest.java    |  33 --
 .../near/GridCacheExNearFullApiSelfTest.java    |  39 ---
 .../GridCacheExReplicatedFullApiSelfTest.java   |  33 --
 .../local/GridCacheExLocalFullApiSelfTest.java  |  30 --
 .../loadtests/hashmap/GridCacheTestContext.java |   1 +
 .../IgniteCacheFullApiSelfTestSuite.java        |   6 -
 .../testsuites/IgniteCacheTestSuite4.java       |   2 +
 24 files changed, 591 insertions(+), 259 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/0a4e7dd4/modules/core/src/main/java/org/apache/ignite/internal/IgniteKernal.java
----------------------------------------------------------------------

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/0a4e7dd4/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheAdapter.java
----------------------------------------------------------------------

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


[20/42] incubator-ignite git commit: Merge remote-tracking branch 'remotes/origin/ignite-sprint-5' into ignite-883_1

Posted by ag...@apache.org.
Merge remote-tracking branch 'remotes/origin/ignite-sprint-5' into ignite-883_1


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

Branch: refs/heads/ignite-389
Commit: db576526600b965d14fe271ac4624401a5b4c239
Parents: a676152 ead66e7
Author: sboikov <sb...@gridgain.com>
Authored: Fri Jun 5 10:02:10 2015 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Fri Jun 5 10:02:10 2015 +0300

----------------------------------------------------------------------
 .../ignite/internal/visor/cache/VisorCache.java  |  2 +-
 .../VisorCacheConfigurationCollectorJob.java     |  6 +++---
 .../internal/visor/cache/VisorCacheMetrics.java  | 19 +++++++++++--------
 .../cache/VisorCacheMetricsCollectorTask.java    | 10 ++++++----
 .../cache/VisorCacheStoreConfiguration.java      |  5 ++---
 5 files changed, 23 insertions(+), 19 deletions(-)
----------------------------------------------------------------------



[04/42] incubator-ignite git commit: # IGNITE-883 Create and use GridTimerProcessor.schedule()

Posted by ag...@apache.org.
# IGNITE-883 Create and use GridTimerProcessor.schedule()


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

Branch: refs/heads/ignite-389
Commit: 997d65e2aeb5d1c7a62a95120876238b0de6d3d4
Parents: bfae889
Author: sevdokimov <se...@gridgain.com>
Authored: Mon Jun 1 18:10:28 2015 +0300
Committer: sevdokimov <se...@gridgain.com>
Committed: Mon Jun 1 19:50:46 2015 +0300

----------------------------------------------------------------------
 .../apache/ignite/internal/IgniteKernal.java    |  28 +----
 .../discovery/GridDiscoveryManager.java         |  28 ++---
 .../timeout/GridTimeoutProcessor.java           | 107 ++++++++++++++++++-
 3 files changed, 117 insertions(+), 46 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/997d65e2/modules/core/src/main/java/org/apache/ignite/internal/IgniteKernal.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/IgniteKernal.java b/modules/core/src/main/java/org/apache/ignite/internal/IgniteKernal.java
index 8a7dc70..e3fc50f 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/IgniteKernal.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/IgniteKernal.java
@@ -167,14 +167,6 @@ public class IgniteKernal implements IgniteEx, IgniteMXBean, Externalizable {
     @GridToStringExclude
     private Timer updateNtfTimer;
 
-    /** */
-    @GridToStringExclude
-    private Timer starveTimer;
-
-    /** */
-    @GridToStringExclude
-    private Timer metricsLogTimer;
-
     /** Indicate error on grid stop. */
     @GridToStringExclude
     private boolean errOnStop;
@@ -867,13 +859,11 @@ public class IgniteKernal implements IgniteEx, IgniteMXBean, Externalizable {
         if (starveCheck) {
             final long interval = F.isEmpty(intervalStr) ? PERIODIC_STARVATION_CHECK_FREQ : Long.parseLong(intervalStr);
 
-            starveTimer = new Timer("ignite-starvation-checker");
-
-            starveTimer.scheduleAtFixedRate(new GridTimerTask() {
+            ctx.timeout().schedule(new Runnable() {
                 /** Last completed task count. */
                 private long lastCompletedCnt;
 
-                @Override protected void safeRun() {
+                @Override public void run() {
                     if (!(execSvc instanceof ThreadPoolExecutor))
                         return;
 
@@ -896,13 +886,10 @@ public class IgniteKernal implements IgniteEx, IgniteMXBean, Externalizable {
         long metricsLogFreq = cfg.getMetricsLogFrequency();
 
         if (metricsLogFreq > 0) {
-            metricsLogTimer = new Timer("ignite-metrics-logger");
-
-            metricsLogTimer.scheduleAtFixedRate(new GridTimerTask() {
-                /** */
+            ctx.timeout().schedule(new Runnable() {
                 private final DecimalFormat dblFmt = new DecimalFormat("#.##");
 
-                @Override protected void safeRun() {
+                @Override public void run() {
                     if (log.isInfoEnabled()) {
                         ClusterMetrics m = cluster().localNode().metrics();
 
@@ -1713,13 +1700,6 @@ public class IgniteKernal implements IgniteEx, IgniteMXBean, Externalizable {
             if (updateNtfTimer != null)
                 updateNtfTimer.cancel();
 
-            if (starveTimer != null)
-                starveTimer.cancel();
-
-            // Cancel metrics log timer.
-            if (metricsLogTimer != null)
-                metricsLogTimer.cancel();
-
             boolean interrupted = false;
 
             while (true) {

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/997d65e2/modules/core/src/main/java/org/apache/ignite/internal/managers/discovery/GridDiscoveryManager.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/managers/discovery/GridDiscoveryManager.java b/modules/core/src/main/java/org/apache/ignite/internal/managers/discovery/GridDiscoveryManager.java
index 4ef602e..9b8280e 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/managers/discovery/GridDiscoveryManager.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/managers/discovery/GridDiscoveryManager.java
@@ -30,6 +30,7 @@ import org.apache.ignite.internal.processors.affinity.*;
 import org.apache.ignite.internal.processors.cache.*;
 import org.apache.ignite.internal.processors.jobmetrics.*;
 import org.apache.ignite.internal.processors.security.*;
+import org.apache.ignite.internal.processors.timeout.*;
 import org.apache.ignite.internal.util.*;
 import org.apache.ignite.internal.util.future.*;
 import org.apache.ignite.internal.util.lang.*;
@@ -165,7 +166,7 @@ public class GridDiscoveryManager extends GridManagerAdapter<DiscoverySpi> {
     private final GridLocalMetrics metrics = createMetrics();
 
     /** Metrics update worker. */
-    private final MetricsUpdater metricsUpdater = new MetricsUpdater();
+    private GridTimeoutProcessor.CancelableTask metricsUpdateTask;
 
     /** Custom event listener. */
     private ConcurrentMap<Class<?>, List<CustomEventListener<DiscoveryCustomMessage>>> customEvtLsnrs =
@@ -325,7 +326,7 @@ public class GridDiscoveryManager extends GridManagerAdapter<DiscoverySpi> {
             checkSegmentOnStart();
         }
 
-        new IgniteThread(metricsUpdater).start();
+        metricsUpdateTask = ctx.timeout().schedule(new MetricsUpdater(), METRICS_UPDATE_FREQ, METRICS_UPDATE_FREQ);
 
         spi.setMetricsProvider(createMetricsProvider());
 
@@ -987,11 +988,11 @@ public class GridDiscoveryManager extends GridManagerAdapter<DiscoverySpi> {
         getSpi().setListener(null);
 
         // Stop discovery worker and metrics updater.
+        U.closeQuiet(metricsUpdateTask);
+
         U.cancel(discoWrk);
-        U.cancel(metricsUpdater);
 
         U.join(discoWrk, log);
-        U.join(metricsUpdater, log);
 
         // Stop SPI itself.
         stopSpi();
@@ -1879,28 +1880,17 @@ public class GridDiscoveryManager extends GridManagerAdapter<DiscoverySpi> {
     /**
      *
      */
-    private class MetricsUpdater extends GridWorker {
+    private class MetricsUpdater implements Runnable {
         /** */
         private long prevGcTime = -1;
 
         /** */
         private long prevCpuTime = -1;
 
-        /**
-         *
-         */
-        private MetricsUpdater() {
-            super(ctx.gridName(), "metrics-updater", GridDiscoveryManager.this.log);
-        }
-
         /** {@inheritDoc} */
-        @Override protected void body() throws IgniteInterruptedCheckedException {
-            while (!isCancelled()) {
-                U.sleep(METRICS_UPDATE_FREQ);
-
-                gcCpuLoad = getGcCpuLoad();
-                cpuLoad = getCpuLoad();
-            }
+        @Override public void run() {
+            gcCpuLoad = getGcCpuLoad();
+            cpuLoad = getCpuLoad();
         }
 
         /**

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/997d65e2/modules/core/src/main/java/org/apache/ignite/internal/processors/timeout/GridTimeoutProcessor.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/timeout/GridTimeoutProcessor.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/timeout/GridTimeoutProcessor.java
index 81ff72b..e9b7717 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/timeout/GridTimeoutProcessor.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/timeout/GridTimeoutProcessor.java
@@ -24,8 +24,10 @@ import org.apache.ignite.internal.util.*;
 import org.apache.ignite.internal.util.typedef.*;
 import org.apache.ignite.internal.util.typedef.internal.*;
 import org.apache.ignite.internal.util.worker.*;
+import org.apache.ignite.lang.*;
 import org.apache.ignite.thread.*;
 
+import java.io.*;
 import java.util.*;
 
 /**
@@ -40,10 +42,12 @@ public class GridTimeoutProcessor extends GridProcessorAdapter {
         new GridConcurrentSkipListSet<>(new Comparator<GridTimeoutObject>() {
             /** {@inheritDoc} */
             @Override public int compare(GridTimeoutObject o1, GridTimeoutObject o2) {
-                long time1 = o1.endTime();
-                long time2 = o2.endTime();
+                int res = Long.compare(o1.endTime(), o2.endTime());
 
-                return time1 < time2 ? -1 : time1 > time2 ? 1 : o1.timeoutId().compareTo(o2.timeoutId());
+                if (res != 0)
+                    return res;
+
+                return o1.timeoutId().compareTo(o2.timeoutId());
             }
         });
 
@@ -98,6 +102,26 @@ public class GridTimeoutProcessor extends GridProcessorAdapter {
     }
 
     /**
+     * Schedule the specified timer task for execution at the specified
+     * time with the specified period, in milliseconds.
+     *
+     * @param task Task to execute.
+     * @param delay Delay to first execution in milliseconds.
+     * @param period Period for execution in milliseconds or -1.
+     * @return Cancelable to cancel task.
+     */
+    public CancelableTask schedule(Runnable task, long delay, long period) {
+        assert delay >= 0;
+        assert period > 0 || period == -1;
+
+        CancelableTask obj = new CancelableTask(task, U.currentTimeMillis() + delay, period);
+
+        addTimeoutObject(obj);
+
+        return obj;
+    }
+
+    /**
      * @param timeoutObj Timeout object.
      */
     public void removeTimeoutObject(GridTimeoutObject timeoutObj) {
@@ -173,4 +197,81 @@ public class GridTimeoutProcessor extends GridProcessorAdapter {
         X.println(">>> Timeout processor memory stats [grid=" + ctx.gridName() + ']');
         X.println(">>>   timeoutObjsSize: " + timeoutObjs.size());
     }
+
+    /**
+     *
+     */
+    public class CancelableTask implements GridTimeoutObject, Closeable {
+        /** */
+        private final IgniteUuid id = new IgniteUuid();
+
+        /** */
+        private long endTime;
+
+        /** */
+        private final long period;
+
+        /** */
+        private volatile boolean cancel;
+
+        /** */
+        private final Runnable task;
+
+        /**
+         * @param firstTime First time.
+         * @param period Period.
+         * @param task Task to execute.
+         */
+        CancelableTask(Runnable task, long firstTime, long period) {
+            this.task = task;
+            endTime = firstTime;
+            this.period = period;
+        }
+
+        /** {@inheritDoc} */
+        @Override public IgniteUuid timeoutId() {
+            return id;
+        }
+
+        /** {@inheritDoc} */
+        @Override public long endTime() {
+            return endTime;
+        }
+
+        /** {@inheritDoc} */
+        @Override public synchronized void onTimeout() {
+            if (cancel)
+                return;
+
+            long startTime = U.currentTimeMillis();
+
+            try {
+                task.run();
+            }
+            finally {
+                long executionTime = U.currentTimeMillis() - startTime;
+
+                if (executionTime > 10) {
+                    U.warn(log, "Timer task take a lot of time, tasks submitted to GridTimeoutProcessor must work " +
+                        "quickly [executionTime=" + executionTime + ']');
+                }
+
+                if (!cancel && period > 0) {
+                    endTime = U.currentTimeMillis() + period;
+
+                    addTimeoutObject(this);
+                }
+            }
+        }
+
+        /** {@inheritDoc} */
+        @Override public void close() {
+            cancel = true;
+
+            synchronized (this) {
+                // Just waiting for task execution end to make sure that task will not be executed anymore.
+                removeTimeoutObject(this);
+            }
+        }
+    }
 }


[34/42] incubator-ignite git commit: # ignite-sprint-5 javadoc warning

Posted by ag...@apache.org.
# ignite-sprint-5 javadoc warning


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

Branch: refs/heads/ignite-389
Commit: a5b5ec703dbcb593a945c8eb1d37e1261e55e834
Parents: 59db4a5
Author: sboikov <sb...@gridgain.com>
Authored: Mon Jun 8 11:32:14 2015 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Mon Jun 8 11:32:14 2015 +0300

----------------------------------------------------------------------
 modules/core/src/main/java/org/apache/ignite/IgniteCache.java | 7 ++++++-
 1 file changed, 6 insertions(+), 1 deletion(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/a5b5ec70/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 07f4f3f..a54adc9 100644
--- a/modules/core/src/main/java/org/apache/ignite/IgniteCache.java
+++ b/modules/core/src/main/java/org/apache/ignite/IgniteCache.java
@@ -325,7 +325,12 @@ public interface IgniteCache<K, V> extends javax.cache.Cache<K, V>, IgniteAsyncS
     @IgniteAsyncSupported
     @Override public boolean containsKey(K key);
 
-    /** {@inheritDoc} */
+    /**
+     * Determines if the {@link Cache} contains entries for the specified keys.
+     *
+     * @param keys Key whose presence in this cache is to be tested.
+     * @return {@code True} if this cache contains a mapping for the specified keys.
+     */
     @IgniteAsyncSupported
     public boolean containsKeys(Set<? extends K> keys);
 


[42/42] incubator-ignite git commit: IGNITE-389 - Merge branch ignite-sprint-5 into ignite-389

Posted by ag...@apache.org.
IGNITE-389 - Merge branch ignite-sprint-5 into ignite-389


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

Branch: refs/heads/ignite-389
Commit: 1552a4b2474bc92ed3aa654c2a477cf444d88e0a
Parents: fa97def 0fa2853
Author: Alexey Goncharuk <ag...@gridgain.com>
Authored: Mon Jun 8 15:27:35 2015 -0700
Committer: Alexey Goncharuk <ag...@gridgain.com>
Committed: Mon Jun 8 15:27:35 2015 -0700

----------------------------------------------------------------------
 DEVNOTES.txt                                    |  42 +-
 .../java/org/apache/ignite/IgniteCache.java     |  16 +
 .../apache/ignite/internal/IgniteKernal.java    |  28 +-
 .../ignite/internal/IgniteNodeAttributes.java   |   5 +-
 .../org/apache/ignite/internal/IgnitionEx.java  |   8 +-
 .../internal/MarshallerContextAdapter.java      |  36 +-
 .../internal/managers/GridManagerAdapter.java   |   9 +
 .../checkpoint/GridCheckpointManager.java       |  52 +-
 .../discovery/GridDiscoveryManager.java         |  28 +-
 .../affinity/GridAffinityProcessor.java         |  23 +-
 .../cache/DynamicCacheDescriptor.java           |  17 +
 .../processors/cache/GridCacheAdapter.java      |  21 +-
 .../processors/cache/GridCacheContext.java      |  13 +
 .../GridCachePartitionExchangeManager.java      |  26 +-
 .../processors/cache/GridCacheProcessor.java    |  37 +-
 .../processors/cache/GridCacheProxyImpl.java    |  14 +-
 .../processors/cache/GridCacheTtlManager.java   |   9 +-
 .../processors/cache/IgniteCacheProxy.java      |  23 +
 .../processors/cache/IgniteInternalCache.java   |  11 +-
 .../dht/atomic/GridDhtAtomicCache.java          |  22 +-
 .../dht/preloader/GridDhtForceKeysFuture.java   |  40 +-
 .../GridDhtPartitionsExchangeFuture.java        |  50 +-
 .../transactions/IgniteTxLocalAdapter.java      |  28 +
 .../cache/transactions/IgniteTxManager.java     |   3 -
 .../datastreamer/DataStreamerImpl.java          |  92 ++-
 .../datastructures/DataStructuresProcessor.java | 107 +++-
 .../processors/igfs/IgfsMetaManager.java        |   2 +-
 .../service/GridServiceProcessor.java           |   4 +-
 .../timeout/GridSpiTimeoutObject.java           |  73 +++
 .../timeout/GridTimeoutProcessor.java           | 105 +++-
 .../IgniteTxRollbackCheckedException.java       |   9 +
 .../util/nio/GridCommunicationClient.java       |  30 +-
 .../util/nio/GridNioRecoveryDescriptor.java     |  13 +-
 .../util/nio/GridTcpCommunicationClient.java    | 554 -------------------
 .../util/nio/GridTcpNioCommunicationClient.java |   8 -
 .../ignite/internal/visor/cache/VisorCache.java |   2 +-
 .../VisorCacheConfigurationCollectorJob.java    |   6 +-
 .../internal/visor/cache/VisorCacheMetrics.java |  19 +-
 .../cache/VisorCacheMetricsCollectorTask.java   |  10 +-
 .../cache/VisorCacheStoreConfiguration.java     |   5 +-
 .../org/apache/ignite/spi/IgniteSpiAdapter.java |  27 +-
 .../org/apache/ignite/spi/IgniteSpiContext.java |  10 +
 .../ignite/spi/IgniteSpiTimeoutObject.java      |  44 ++
 .../spi/checkpoint/noop/NoopCheckpointSpi.java  |   3 +-
 .../communication/tcp/TcpCommunicationSpi.java  | 443 ++++-----------
 .../tcp/TcpCommunicationSpiMBean.java           |   2 -
 .../ignite/spi/discovery/tcp/ClientImpl.java    |   3 -
 .../ignite/spi/discovery/tcp/ServerImpl.java    |  10 +-
 .../spi/discovery/tcp/TcpDiscoverySpi.java      | 156 +-----
 ...acheReadOnlyTransactionalClientSelfTest.java | 327 +++++++++++
 .../cache/GridCacheAbstractFullApiSelfTest.java |  83 +++
 .../GridCacheExAbstractFullApiSelfTest.java     | 103 ----
 .../IgniteCountDownLatchAbstractSelfTest.java   | 102 ++++
 .../GridCacheExColocatedFullApiSelfTest.java    |  33 --
 .../near/GridCacheExNearFullApiSelfTest.java    |  39 --
 .../GridCacheExReplicatedFullApiSelfTest.java   |  33 --
 .../IgniteCacheClientNearCacheExpiryTest.java   | 103 ++++
 .../IgniteCacheExpiryPolicyTestSuite.java       |   2 +
 .../local/GridCacheExLocalFullApiSelfTest.java  |  30 -
 .../DataStreamerMultiThreadedSelfTest.java      |  59 +-
 .../DataStreamerMultinodeCreateCacheTest.java   |  97 ++++
 .../internal/util/nio/GridNioSelfTest.java      |   2 +-
 .../loadtests/hashmap/GridCacheTestContext.java |   1 +
 .../IgniteMessagingWithClientTest.java          |   2 +
 .../GridTcpCommunicationSpiAbstractTest.java    |   4 +-
 ...mmunicationSpiConcurrentConnectSelfTest.java |   2 +-
 .../GridTcpCommunicationSpiConfigSelfTest.java  |   2 -
 ...cpCommunicationSpiMultithreadedSelfTest.java |   2 +-
 .../discovery/AbstractDiscoverySelfTest.java    |  13 +-
 .../tcp/TcpClientDiscoverySpiSelfTest.java      |  25 +
 .../testframework/GridSpiTestContext.java       |  10 +
 .../IgniteCacheFullApiSelfTestSuite.java        |   6 -
 .../ignite/testsuites/IgniteCacheTestSuite.java |   1 +
 .../testsuites/IgniteCacheTestSuite4.java       |   2 +
 74 files changed, 1825 insertions(+), 1556 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1552a4b2/DEVNOTES.txt
----------------------------------------------------------------------

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1552a4b2/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheAdapter.java
----------------------------------------------------------------------

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1552a4b2/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheProcessor.java
----------------------------------------------------------------------

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

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1552a4b2/modules/core/src/main/java/org/apache/ignite/internal/processors/service/GridServiceProcessor.java
----------------------------------------------------------------------

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1552a4b2/modules/core/src/main/java/org/apache/ignite/spi/communication/tcp/TcpCommunicationSpi.java
----------------------------------------------------------------------
diff --cc modules/core/src/main/java/org/apache/ignite/spi/communication/tcp/TcpCommunicationSpi.java
index 3768db5,359de1c..a661965
--- 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
@@@ -691,23 -634,8 +678,14 @@@ public class TcpCommunicationSpi extend
      /** Socket write timeout. */
      private long sockWriteTimeout = DFLT_SOCK_WRITE_TIMEOUT;
  
+     /** Recovery and idle clients handler. */
+     private CommunicationWorker commWorker;
++    
 +    /** Shared memory accept worker. */
 +    private ShmemAcceptWorker shmemAcceptWorker;
 +
-     /** Idle client worker. */
-     private IdleClientWorker idleClientWorker;
- 
-     /** Flush client worker. */
-     private ClientFlushWorker clientFlushWorker;
- 
-     /** Socket timeout worker. */
-     private SocketTimeoutWorker sockTimeoutWorker;
- 
-     /** Recovery worker. */
-     private RecoveryWorker recoveryWorker;
- 
 +    /** Shared memory workers. */
 +    private final Collection<ShmemWorker> shmemWorkers = new ConcurrentLinkedDeque8<>();
  
      /** Clients. */
      private final ConcurrentMap<UUID, GridCommunicationClient> clients = GridConcurrentFactory.newMap();
@@@ -1354,31 -1239,11 +1321,17 @@@
  
          registerMBean(gridName, this, TcpCommunicationSpiMBean.class);
  
 +        if (shmemSrv != null) {
 +            shmemAcceptWorker = new ShmemAcceptWorker(shmemSrv);
 +
 +            new IgniteThread(shmemAcceptWorker).start();
 +        }
 +
          nioSrvr.start();
  
-         idleClientWorker = new IdleClientWorker();
+         commWorker = new CommunicationWorker();
  
-         idleClientWorker.start();
- 
-         recoveryWorker = new RecoveryWorker();
- 
-         recoveryWorker.start();
- 
-         if (connBufSize > 0) {
-             clientFlushWorker = new ClientFlushWorker();
- 
-             clientFlushWorker.start();
-         }
- 
-         sockTimeoutWorker = new SocketTimeoutWorker();
- 
-         sockTimeoutWorker.start();
+         commWorker.start();
  
          // Ack start.
          if (log.isDebugEnabled())
@@@ -1586,24 -1398,10 +1539,17 @@@
          if (nioSrvr != null)
              nioSrvr.stop();
  
+         U.interrupt(commWorker);
 -
+         U.join(commWorker, log);
+ 
 +        U.cancel(shmemAcceptWorker);
 +        U.join(shmemAcceptWorker, log);
 +
-         U.interrupt(idleClientWorker);
-         U.interrupt(clientFlushWorker);
-         U.interrupt(sockTimeoutWorker);
-         U.interrupt(recoveryWorker);
- 
-         U.join(idleClientWorker, log);
-         U.join(clientFlushWorker, log);
-         U.join(sockTimeoutWorker, log);
-         U.join(recoveryWorker, log);
- 
 +        U.cancel(shmemWorkers);
 +        U.join(shmemWorkers, log);
 +
 +        shmemWorkers.clear();
 +
          // Force closing on stop (safety).
          for (GridCommunicationClient client : clients.values())
              client.forceClose();
@@@ -2400,147 -2095,12 +2340,150 @@@
      }
  
      /**
 +     * This worker takes responsibility to shut the server down when stopping,
 +     * No other thread shall stop passed server.
 +     */
 +    private class ShmemAcceptWorker extends GridWorker {
 +        /** */
 +        private final IpcSharedMemoryServerEndpoint srv;
 +
 +        /**
 +         * @param srv Server.
 +         */
 +        ShmemAcceptWorker(IpcSharedMemoryServerEndpoint srv) {
 +            super(gridName, "shmem-communication-acceptor", TcpCommunicationSpi.this.log);
 +
 +            this.srv = srv;
 +        }
 +
 +        /** {@inheritDoc} */
 +        @Override protected void body() throws InterruptedException {
 +            try {
 +                while (!Thread.interrupted()) {
 +                    ShmemWorker e = new ShmemWorker(srv.accept());
 +
 +                    shmemWorkers.add(e);
 +
 +                    new IgniteThread(e).start();
 +                }
 +            }
 +            catch (IgniteCheckedException e) {
 +                if (!isCancelled())
 +                    U.error(log, "Shmem server failed.", e);
 +            }
 +            finally {
 +                srv.close();
 +            }
 +        }
 +
 +        /** {@inheritDoc} */
 +        @Override public void cancel() {
 +            super.cancel();
 +
 +            srv.close();
 +        }
 +    }
 +
 +    /**
 +     *
 +     */
 +    private class ShmemWorker extends GridWorker {
 +        /** */
 +        private final IpcEndpoint endpoint;
 +
 +        /**
 +         * @param endpoint Endpoint.
 +         */
 +        private ShmemWorker(IpcEndpoint endpoint) {
 +            super(gridName, "shmem-worker", TcpCommunicationSpi.this.log);
 +
 +            this.endpoint = endpoint;
 +        }
 +
 +        /** {@inheritDoc} */
 +        @Override protected void body() throws InterruptedException {
 +            try {
 +                MessageFactory msgFactory = new MessageFactory() {
 +                    private MessageFactory impl;
 +
 +                    @Nullable @Override public Message create(byte type) {
 +                        if (impl == null)
 +                            impl = getSpiContext().messageFactory();
 +
 +                        assert impl != null;
 +
 +                        return impl.create(type);
 +                    }
 +                };
 +
 +                MessageFormatter msgFormatter = new MessageFormatter() {
 +                    private MessageFormatter impl;
 +
 +                    @Override public MessageWriter writer() {
 +                        if (impl == null)
 +                            impl = getSpiContext().messageFormatter();
 +
 +                        assert impl != null;
 +
 +                        return impl.writer();
 +                    }
 +
 +                    @Override public MessageReader reader(MessageFactory factory) {
 +                        if (impl == null)
 +                            impl = getSpiContext().messageFormatter();
 +
 +                        assert impl != null;
 +
 +                        return impl.reader(factory);
 +                    }
 +                };
 +
 +                IpcToNioAdapter<Message> adapter = new IpcToNioAdapter<>(
 +                    metricsLsnr,
 +                    log,
 +                    endpoint,
 +                    srvLsnr,
 +                    msgFormatter,
 +                    new GridNioCodecFilter(new GridDirectParser(msgFactory, msgFormatter), log, true),
 +                    new GridConnectionBytesVerifyFilter(log)
 +                );
 +
 +                adapter.serve();
 +            }
 +            finally {
 +                shmemWorkers.remove(this);
 +
 +                endpoint.close();
 +            }
 +        }
 +
 +        /** {@inheritDoc} */
 +        @Override public void cancel() {
 +            super.cancel();
 +
 +            endpoint.close();
 +        }
 +
 +        /** @{@inheritDoc} */
 +        @Override protected void cleanup() {
 +            super.cleanup();
 +
 +            endpoint.close();
 +        }
 +
 +        /** @{@inheritDoc} */
 +        @Override public String toString() {
 +            return S.toString(ShmemWorker.class, this);
 +        }
 +    }
 +
 +    /**
       *
       */
-     private class IdleClientWorker extends IgniteSpiThread {
+     private class CommunicationWorker extends IgniteSpiThread {
+         /** */
+         private final BlockingQueue<GridNioRecoveryDescriptor> q = new LinkedBlockingQueue<>();
+ 
          /**
           *
           */

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1552a4b2/modules/core/src/main/java/org/apache/ignite/spi/communication/tcp/TcpCommunicationSpiMBean.java
----------------------------------------------------------------------

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1552a4b2/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAbstractFullApiSelfTest.java
----------------------------------------------------------------------

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1552a4b2/modules/core/src/test/java/org/apache/ignite/spi/communication/tcp/GridTcpCommunicationSpiAbstractTest.java
----------------------------------------------------------------------

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1552a4b2/modules/core/src/test/java/org/apache/ignite/spi/communication/tcp/GridTcpCommunicationSpiConcurrentConnectSelfTest.java
----------------------------------------------------------------------

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1552a4b2/modules/core/src/test/java/org/apache/ignite/spi/communication/tcp/GridTcpCommunicationSpiMultithreadedSelfTest.java
----------------------------------------------------------------------


[22/42] incubator-ignite git commit: Merge remote-tracking branch 'origin/ignite-sprint-5' into ignite-sprint-5

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


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

Branch: refs/heads/ignite-389
Commit: e87969c1023786e8b5fb7d19057bf715f4d3ecac
Parents: dc1d427 ead66e7
Author: sboikov <sb...@gridgain.com>
Authored: Fri Jun 5 11:34:48 2015 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Fri Jun 5 11:34:48 2015 +0300

----------------------------------------------------------------------
 .../processors/hadoop/HadoopTaskContext.java    |  14 +-
 .../igfs/IgfsSecondaryFileSystemImpl.java       |   2 +-
 .../ignite/internal/visor/cache/VisorCache.java |   2 +-
 .../VisorCacheConfigurationCollectorJob.java    |   6 +-
 .../internal/visor/cache/VisorCacheMetrics.java |  19 +-
 .../cache/VisorCacheMetricsCollectorTask.java   |  10 +-
 .../cache/VisorCacheStoreConfiguration.java     |   5 +-
 .../fs/IgniteHadoopFileSystemCounterWriter.java |  14 +-
 .../hadoop/fs/v1/IgniteHadoopFileSystem.java    |  70 ++---
 .../hadoop/fs/v2/IgniteHadoopFileSystem.java    |   2 +-
 .../processors/hadoop/HadoopDefaultJobInfo.java |   2 +-
 .../internal/processors/hadoop/HadoopUtils.java | 282 ++++++++++++++++++-
 .../hadoop/SecondaryFileSystemProvider.java     |   4 +-
 .../hadoop/taskexecutor/HadoopRunnableTask.java |  20 +-
 .../processors/hadoop/v2/HadoopV2Job.java       |  31 +-
 .../hadoop/v2/HadoopV2JobResourceManager.java   |  26 +-
 .../hadoop/v2/HadoopV2TaskContext.java          |  48 +++-
 .../hadoop/HadoopClientProtocolSelfTest.java    |   6 +-
 .../hadoop/HadoopAbstractSelfTest.java          |  14 +-
 .../hadoop/HadoopCommandLineTest.java           |  14 +-
 .../processors/hadoop/HadoopMapReduceTest.java  | 176 +++++++++++-
 .../hadoop/HadoopTaskExecutionSelfTest.java     |   2 +-
 .../hadoop/HadoopTasksAllVersionsTest.java      |  15 +-
 .../processors/hadoop/HadoopTasksV1Test.java    |   5 +-
 .../processors/hadoop/HadoopTasksV2Test.java    |   5 +-
 .../processors/hadoop/HadoopV2JobSelfTest.java  |   6 +-
 .../collections/HadoopAbstractMapTest.java      |  12 +
 27 files changed, 666 insertions(+), 146 deletions(-)
----------------------------------------------------------------------



[15/42] incubator-ignite git commit: Merge remote-tracking branch 'remotes/origin/ignite-sprint-5' into ignite-883_1

Posted by ag...@apache.org.
Merge remote-tracking branch 'remotes/origin/ignite-sprint-5' into ignite-883_1


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

Branch: refs/heads/ignite-389
Commit: 3417b3dc0904708dcbea6396f88737a2aa99fa79
Parents: 6bfc78e c9f7291
Author: sboikov <se...@inria.fr>
Authored: Thu Jun 4 21:28:32 2015 +0300
Committer: sboikov <se...@inria.fr>
Committed: Thu Jun 4 21:28:32 2015 +0300

----------------------------------------------------------------------
 .../configuration/CacheConfiguration.java       |  13 +-
 .../apache/ignite/internal/IgniteKernal.java    |   3 +
 .../managers/communication/GridIoManager.java   | 117 ++++----
 .../dht/GridClientPartitionTopology.java        |   2 +-
 .../dht/GridDhtPartitionTopologyImpl.java       |   8 +-
 .../GridDhtPartitionsExchangeFuture.java        |  19 +-
 .../dht/preloader/GridDhtPreloader.java         |   2 +-
 .../processors/hadoop/HadoopTaskContext.java    |  14 +-
 .../igfs/IgfsSecondaryFileSystemImpl.java       |   2 +-
 .../internal/visor/query/VisorQueryJob.java     |   2 +-
 ...niteDynamicCacheWithConfigStartSelfTest.java |  97 +++++++
 .../igfs/IgfsClientCacheSelfTest.java           |   9 +-
 .../IgniteMessagingWithClientTest.java          | 164 +++++++++++
 .../ignite/testsuites/IgniteBasicTestSuite.java |   1 +
 .../testsuites/IgniteCacheTestSuite4.java       |   1 +
 .../fs/IgniteHadoopFileSystemCounterWriter.java |  14 +-
 .../hadoop/fs/v1/IgniteHadoopFileSystem.java    |  70 ++---
 .../hadoop/fs/v2/IgniteHadoopFileSystem.java    |   2 +-
 .../processors/hadoop/HadoopDefaultJobInfo.java |   2 +-
 .../internal/processors/hadoop/HadoopUtils.java | 282 ++++++++++++++++++-
 .../hadoop/SecondaryFileSystemProvider.java     |   4 +-
 .../hadoop/taskexecutor/HadoopRunnableTask.java |  20 +-
 .../processors/hadoop/v2/HadoopV2Job.java       |  31 +-
 .../hadoop/v2/HadoopV2JobResourceManager.java   |  26 +-
 .../hadoop/v2/HadoopV2TaskContext.java          |  48 +++-
 .../hadoop/HadoopClientProtocolSelfTest.java    |   6 +-
 .../hadoop/HadoopAbstractSelfTest.java          |  14 +-
 .../hadoop/HadoopCommandLineTest.java           |  14 +-
 .../processors/hadoop/HadoopMapReduceTest.java  | 176 +++++++++++-
 .../hadoop/HadoopTaskExecutionSelfTest.java     |   2 +-
 .../hadoop/HadoopTasksAllVersionsTest.java      |  15 +-
 .../processors/hadoop/HadoopTasksV1Test.java    |   5 +-
 .../processors/hadoop/HadoopTasksV2Test.java    |   5 +-
 .../processors/hadoop/HadoopV2JobSelfTest.java  |   6 +-
 .../collections/HadoopAbstractMapTest.java      |  12 +
 ...acheConfigurationPrimitiveTypesSelfTest.java | 104 +++++++
 .../IgniteCacheWithIndexingTestSuite.java       |   2 +
 .../commands/cache/VisorCacheScanCommand.scala  |   2 +-
 38 files changed, 1122 insertions(+), 194 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/3417b3dc/modules/core/src/main/java/org/apache/ignite/internal/IgniteKernal.java
----------------------------------------------------------------------


[36/42] incubator-ignite git commit: Merge remote-tracking branch 'remotes/origin/ignite-883_1' into ignite-sprint-5

Posted by ag...@apache.org.
Merge remote-tracking branch 'remotes/origin/ignite-883_1' into ignite-sprint-5


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

Branch: refs/heads/ignite-389
Commit: 015afdbf1921e6ae5e8cd2f92efc8b4a695f05ca
Parents: 8467a3c c7bc598
Author: sboikov <sb...@gridgain.com>
Authored: Mon Jun 8 13:38:48 2015 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Mon Jun 8 13:38:48 2015 +0300

----------------------------------------------------------------------
 .../apache/ignite/internal/IgniteKernal.java    |  26 +-
 .../org/apache/ignite/internal/IgnitionEx.java  |   8 +-
 .../internal/managers/GridManagerAdapter.java   |   9 +
 .../checkpoint/GridCheckpointManager.java       |  52 +-
 .../discovery/GridDiscoveryManager.java         |  28 +-
 .../processors/cache/GridCacheAdapter.java      |   4 +
 .../GridCachePartitionExchangeManager.java      |  26 +-
 .../processors/cache/GridCacheTtlManager.java   |   9 +-
 .../dht/atomic/GridDhtAtomicCache.java          |  18 +-
 .../GridDhtPartitionsExchangeFuture.java        |  50 +-
 .../cache/transactions/IgniteTxManager.java     |   3 -
 .../datastructures/DataStructuresProcessor.java | 107 +++-
 .../service/GridServiceProcessor.java           |   4 +-
 .../timeout/GridSpiTimeoutObject.java           |  73 +++
 .../timeout/GridTimeoutProcessor.java           | 105 +++-
 .../util/nio/GridCommunicationClient.java       |  30 +-
 .../util/nio/GridNioRecoveryDescriptor.java     |  13 +-
 .../util/nio/GridTcpCommunicationClient.java    | 554 -------------------
 .../util/nio/GridTcpNioCommunicationClient.java |   8 -
 .../org/apache/ignite/spi/IgniteSpiAdapter.java |  27 +-
 .../org/apache/ignite/spi/IgniteSpiContext.java |  10 +
 .../ignite/spi/IgniteSpiTimeoutObject.java      |  44 ++
 .../spi/checkpoint/noop/NoopCheckpointSpi.java  |   3 +-
 .../communication/tcp/TcpCommunicationSpi.java  | 438 ++++-----------
 .../tcp/TcpCommunicationSpiMBean.java           |   2 -
 .../ignite/spi/discovery/tcp/ClientImpl.java    |   3 -
 .../ignite/spi/discovery/tcp/ServerImpl.java    |  10 +-
 .../spi/discovery/tcp/TcpDiscoverySpi.java      | 156 +-----
 .../IgniteCountDownLatchAbstractSelfTest.java   | 102 ++++
 .../IgniteCacheClientNearCacheExpiryTest.java   | 103 ++++
 .../IgniteCacheExpiryPolicyTestSuite.java       |   2 +
 .../internal/util/nio/GridNioSelfTest.java      |   2 +-
 .../GridTcpCommunicationSpiAbstractTest.java    |   4 +-
 ...mmunicationSpiConcurrentConnectSelfTest.java |   2 +-
 .../GridTcpCommunicationSpiConfigSelfTest.java  |   2 -
 ...cpCommunicationSpiMultithreadedSelfTest.java |   2 +-
 .../discovery/AbstractDiscoverySelfTest.java    |  13 +-
 .../tcp/TcpClientDiscoverySpiSelfTest.java      |  25 +
 .../testframework/GridSpiTestContext.java       |  10 +
 39 files changed, 886 insertions(+), 1201 deletions(-)
----------------------------------------------------------------------



[17/42] incubator-ignite git commit: IGNITE-941 - Read-only mode for transactional cache

Posted by ag...@apache.org.
IGNITE-941 - Read-only mode for transactional cache


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

Branch: refs/heads/ignite-389
Commit: b4d19159a76559152435946d1fff421679712599
Parents: b58c892
Author: Valentin Kulichenko <vk...@gridgain.com>
Authored: Thu Jun 4 18:08:35 2015 -0700
Committer: Valentin Kulichenko <vk...@gridgain.com>
Committed: Thu Jun 4 18:08:35 2015 -0700

----------------------------------------------------------------------
 .../apache/ignite/internal/IgniteKernal.java    |  2 +
 .../ignite/internal/IgniteNodeAttributes.java   |  5 +-
 .../processors/cache/GridCacheProcessor.java    |  2 +-
 .../transactions/IgniteTxLocalAdapter.java      | 29 +++++++--
 .../IgniteTxRollbackCheckedException.java       |  9 +++
 ...acheReadOnlyTransactionalClientSelfTest.java | 68 +++++++++++++++-----
 6 files changed, 94 insertions(+), 21 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/b4d19159/modules/core/src/main/java/org/apache/ignite/internal/IgniteKernal.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/IgniteKernal.java b/modules/core/src/main/java/org/apache/ignite/internal/IgniteKernal.java
index a0d97c7..9bc6cce 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/IgniteKernal.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/IgniteKernal.java
@@ -1168,6 +1168,8 @@ public class IgniteKernal implements IgniteEx, IgniteMXBean, Externalizable {
 
         add(ATTR_CLIENT_MODE, cfg.isClientMode());
 
+        add(ATTR_CONSISTENCY_CHECK_SKIPPED, getBoolean(IGNITE_SKIP_CONFIGURATION_CONSISTENCY_CHECK));
+
         // Build a string from JVM arguments, because parameters with spaces are split.
         SB jvmArgs = new SB(512);
 

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/b4d19159/modules/core/src/main/java/org/apache/ignite/internal/IgniteNodeAttributes.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/IgniteNodeAttributes.java b/modules/core/src/main/java/org/apache/ignite/internal/IgniteNodeAttributes.java
index 98cc3a7..928db5e 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/IgniteNodeAttributes.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/IgniteNodeAttributes.java
@@ -126,9 +126,12 @@ public final class IgniteNodeAttributes {
     /** Security subject for authenticated node. */
     public static final String ATTR_SECURITY_SUBJECT = ATTR_PREFIX + ".security.subject";
 
-    /** Cache interceptors. */
+    /** Client mode flag. */
     public static final String ATTR_CLIENT_MODE = ATTR_PREFIX + ".cache.client";
 
+    /** Configuration consistency check disabled flag. */
+    public static final String ATTR_CONSISTENCY_CHECK_SKIPPED = ATTR_PREFIX + ".consistency.check.skipped";
+
     /**
      * Enforces singleton.
      */

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/b4d19159/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheProcessor.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheProcessor.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheProcessor.java
index bbd5fe9..871cd77 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheProcessor.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheProcessor.java
@@ -674,7 +674,7 @@ public class GridCacheProcessor extends GridProcessorAdapter {
 
             if (!getBoolean(IGNITE_SKIP_CONFIGURATION_CONSISTENCY_CHECK)) {
                 for (ClusterNode n : ctx.discovery().remoteNodes()) {
-                    if (Boolean.valueOf(n.<String>attribute(IGNITE_SKIP_CONFIGURATION_CONSISTENCY_CHECK)))
+                    if (n.attribute(ATTR_CONSISTENCY_CHECK_SKIPPED))
                         continue;
 
                     checkTransactionConfiguration(n);

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/b4d19159/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/transactions/IgniteTxLocalAdapter.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/transactions/IgniteTxLocalAdapter.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/transactions/IgniteTxLocalAdapter.java
index 4063be2..8b5eaec 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/transactions/IgniteTxLocalAdapter.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/transactions/IgniteTxLocalAdapter.java
@@ -2535,6 +2535,19 @@ public abstract class IgniteTxLocalAdapter extends IgniteTxAdapter
     }
 
     /**
+     * @param cacheCtx Cache context.
+     * @throws IgniteCheckedException If updates are not allowed.
+     */
+    private void checkUpdatesAllowed(GridCacheContext cacheCtx) throws IgniteCheckedException {
+        if (!cacheCtx.updatesAllowed()) {
+            throw new IgniteTxRollbackCheckedException(new CacheException(
+                "Updates are not allowed for transactional cache: " + cacheCtx.name() + ". Configure " +
+                "persistence store on client or use remote closure execution to start transactions " +
+                "from server nodes."));
+        }
+    }
+
+    /**
      * Internal method for all put and transform operations. Only one of {@code map}, {@code transformMap}
      * maps must be non-null.
      *
@@ -2561,8 +2574,12 @@ public abstract class IgniteTxLocalAdapter extends IgniteTxAdapter
     ) {
         assert filter == null || invokeMap == null;
 
-        if (!cacheCtx.updatesAllowed())
-            throw new CacheException("Updates are not allowed for cache: " + cacheCtx.name());
+        try {
+            checkUpdatesAllowed(cacheCtx);
+        }
+        catch (IgniteCheckedException e) {
+            return new GridFinishedFuture(e);
+        }
 
         cacheCtx.checkSecurity(SecurityPermission.CACHE_PUT);
 
@@ -2784,8 +2801,12 @@ public abstract class IgniteTxLocalAdapter extends IgniteTxAdapter
         @Nullable GridCacheEntryEx cached,
         final boolean retval,
         @Nullable final CacheEntryPredicate[] filter) {
-        if (!cacheCtx.updatesAllowed())
-            throw new CacheException("Updates are not allowed for cache: " + cacheCtx.name());
+        try {
+            checkUpdatesAllowed(cacheCtx);
+        }
+        catch (IgniteCheckedException e) {
+            return new GridFinishedFuture(e);
+        }
 
         cacheCtx.checkSecurity(SecurityPermission.CACHE_REMOVE);
 

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/b4d19159/modules/core/src/main/java/org/apache/ignite/internal/transactions/IgniteTxRollbackCheckedException.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/transactions/IgniteTxRollbackCheckedException.java b/modules/core/src/main/java/org/apache/ignite/internal/transactions/IgniteTxRollbackCheckedException.java
index 1d1e022..f8ee265 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/transactions/IgniteTxRollbackCheckedException.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/transactions/IgniteTxRollbackCheckedException.java
@@ -36,6 +36,15 @@ public class IgniteTxRollbackCheckedException extends IgniteCheckedException {
     }
 
     /**
+     * Creates new exception with given nested exception.
+     *
+     * @param cause Nested exception.
+     */
+    public IgniteTxRollbackCheckedException(Throwable cause) {
+        super(cause);
+    }
+
+    /**
      * Creates new rollback exception with given error message and optional nested exception.
      *
      * @param msg Error message.

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/b4d19159/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheReadOnlyTransactionalClientSelfTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheReadOnlyTransactionalClientSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheReadOnlyTransactionalClientSelfTest.java
index e2a1def..f2c38e1 100644
--- a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheReadOnlyTransactionalClientSelfTest.java
+++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheReadOnlyTransactionalClientSelfTest.java
@@ -31,6 +31,8 @@ import javax.cache.*;
 import javax.cache.configuration.*;
 import javax.cache.processor.*;
 
+import static org.apache.ignite.IgniteSystemProperties.*;
+
 /**
  * Tests for read-only transactional cache client.
  */
@@ -39,6 +41,9 @@ public class CacheReadOnlyTransactionalClientSelfTest extends GridCommonAbstract
     private static final TcpDiscoveryIpFinder IP_FINDER = new TcpDiscoveryVmIpFinder(true);
 
     /** */
+    private static final String CACHE_NAME = "test-cache";
+
+    /** */
     private boolean client;
 
     /** */
@@ -55,6 +60,7 @@ public class CacheReadOnlyTransactionalClientSelfTest extends GridCommonAbstract
 
         CacheConfiguration cc = new CacheConfiguration();
 
+        cc.setName(CACHE_NAME);
         cc.setAtomicityMode(CacheAtomicityMode.TRANSACTIONAL);
         cc.setCacheStoreFactory(factory);
 
@@ -100,7 +106,7 @@ public class CacheReadOnlyTransactionalClientSelfTest extends GridCommonAbstract
 
         Ignite ignite = startGrid();
 
-        IgniteCache cache = ignite.cache(null);
+        IgniteCache cache = ignite.cache(CACHE_NAME);
 
         cache.get(0);
         cache.getAll(F.asSet(0, 1));
@@ -140,6 +146,25 @@ public class CacheReadOnlyTransactionalClientSelfTest extends GridCommonAbstract
     /**
      * @throws Exception If failed.
      */
+    public void testDisabledConsistencyCheck() throws Exception {
+        client = false;
+        nearEnabled = false;
+        factory = new Factory2();
+
+        System.setProperty(IGNITE_SKIP_CONFIGURATION_CONSISTENCY_CHECK, "true");
+
+        startGrid("client-1");
+
+        factory = new Factory1();
+
+        System.clearProperty(IGNITE_SKIP_CONFIGURATION_CONSISTENCY_CHECK);
+
+        startGrid("client-2");
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
     public void testNoStoreNearDisabled() throws Exception {
         nearEnabled = false;
 
@@ -164,7 +189,7 @@ public class CacheReadOnlyTransactionalClientSelfTest extends GridCommonAbstract
 
         Ignite ignite = startGrid();
 
-        IgniteCache cache = ignite.cache(null);
+        IgniteCache cache = ignite.cache(CACHE_NAME);
 
         cache.get(0);
         cache.getAll(F.asSet(0, 1));
@@ -173,91 +198,104 @@ public class CacheReadOnlyTransactionalClientSelfTest extends GridCommonAbstract
             cache.getAndPut(0, 0);
         }
         catch (CacheException e) {
-            assertEquals("Updates are not allowed for cache: null", e.getMessage());
+            assert e.getMessage().startsWith("Updates are not allowed for transactional cache: " + CACHE_NAME + ".") :
+                e.getMessage();
         }
 
         try {
             cache.getAndPutIfAbsent(0, 0);
         }
         catch (CacheException e) {
-            assertEquals("Updates are not allowed for cache: null", e.getMessage());
+            assert e.getMessage().startsWith("Updates are not allowed for transactional cache: " + CACHE_NAME + ".") :
+                e.getMessage();
         }
 
         try {
             cache.getAndRemove(0);
         }
         catch (CacheException e) {
-            assertEquals("Updates are not allowed for cache: null", e.getMessage());
+            assert e.getMessage().startsWith("Updates are not allowed for transactional cache: " + CACHE_NAME + ".") :
+                e.getMessage();
         }
 
         try {
             cache.getAndReplace(0, 0);
         }
         catch (CacheException e) {
-            assertEquals("Updates are not allowed for cache: null", e.getMessage());
+            assert e.getMessage().startsWith("Updates are not allowed for transactional cache: " + CACHE_NAME + ".") :
+                e.getMessage();
         }
 
         try {
             cache.put(0, 0);
         }
         catch (CacheException e) {
-            assertEquals("Updates are not allowed for cache: null", e.getMessage());
+            assert e.getMessage().startsWith("Updates are not allowed for transactional cache: " + CACHE_NAME + ".") :
+                e.getMessage();
         }
 
         try {
             cache.putAll(F.asMap(0, 0, 1, 1));
         }
         catch (CacheException e) {
-            assertEquals("Updates are not allowed for cache: null", e.getMessage());
+            assert e.getMessage().startsWith("Updates are not allowed for transactional cache: " + CACHE_NAME + ".") :
+                e.getMessage();
         }
 
         try {
             cache.putIfAbsent(0, 0);
         }
         catch (CacheException e) {
-            assertEquals("Updates are not allowed for cache: null", e.getMessage());
+            assert e.getMessage().startsWith("Updates are not allowed for transactional cache: " + CACHE_NAME + ".") :
+                e.getMessage();
         }
 
         try {
             cache.remove(0);
         }
         catch (CacheException e) {
-            assertEquals("Updates are not allowed for cache: null", e.getMessage());
+            assert e.getMessage().startsWith("Updates are not allowed for transactional cache: " + CACHE_NAME + ".") :
+                e.getMessage();
         }
 
         try {
             cache.remove(0, 0);
         }
         catch (CacheException e) {
-            assertEquals("Updates are not allowed for cache: null", e.getMessage());
+            assert e.getMessage().startsWith("Updates are not allowed for transactional cache: " + CACHE_NAME + ".") :
+                e.getMessage();
         }
 
         try {
             cache.removeAll(F.asSet(0, 1));
         }
         catch (CacheException e) {
-            assertEquals("Updates are not allowed for cache: null", e.getMessage());
+            assert e.getMessage().startsWith("Updates are not allowed for transactional cache: " + CACHE_NAME + ".") :
+                e.getMessage();
         }
 
         try {
             cache.removeAll();
         }
         catch (CacheException e) {
-            assertEquals("Updates are not allowed for cache: null", e.getMessage());
+            assert e.getMessage().startsWith("Updates are not allowed for transactional cache: " + CACHE_NAME + ".") :
+                e.getMessage();
         }
 
         try {
             cache.invoke(0, new EP());
         }
         catch (CacheException e) {
-            assertEquals("Updates are not allowed for cache: null", e.getMessage());
+            assert e.getMessage().startsWith("Updates are not allowed for transactional cache: " + CACHE_NAME + ".") :
+                e.getMessage();
         }
 
         try {
             cache.invokeAll(F.asSet(0, 1), new EP());
         }
         catch (CacheException e) {
-            assertEquals("Updates are not allowed for cache: null", e.getMessage());
+            assert e.getMessage().startsWith("Updates are not allowed for transactional cache: " + CACHE_NAME + ".") :
+                e.getMessage();
         }
     }
 


[29/42] incubator-ignite git commit: # ignite-883 javadoc

Posted by ag...@apache.org.
# ignite-883 javadoc


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

Branch: refs/heads/ignite-389
Commit: 50b0b49987a99fd10691346244f7c60b8d2b6ddd
Parents: 75006b8
Author: sboikov <sb...@gridgain.com>
Authored: Mon Jun 8 10:05:39 2015 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Mon Jun 8 10:05:39 2015 +0300

----------------------------------------------------------------------
 .../java/org/apache/ignite/spi/IgniteSpiTimeoutObject.java     | 6 +++++-
 1 file changed, 5 insertions(+), 1 deletion(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/50b0b499/modules/core/src/main/java/org/apache/ignite/spi/IgniteSpiTimeoutObject.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/spi/IgniteSpiTimeoutObject.java b/modules/core/src/main/java/org/apache/ignite/spi/IgniteSpiTimeoutObject.java
index f7fbd27f..b3fc28e 100644
--- a/modules/core/src/main/java/org/apache/ignite/spi/IgniteSpiTimeoutObject.java
+++ b/modules/core/src/main/java/org/apache/ignite/spi/IgniteSpiTimeoutObject.java
@@ -20,7 +20,11 @@ package org.apache.ignite.spi;
 import org.apache.ignite.lang.*;
 
 /**
- *
+ * Provides possibility to schedule delayed execution,
+ * see {@link IgniteSpiContext#addTimeoutObject(IgniteSpiTimeoutObject)}.
+ * <p>
+ * Note: all timeout objects are executed in single dedicated thread, so implementation
+ * of {@link #onTimeout()} should not use time consuming and blocking method.
  */
 public interface IgniteSpiTimeoutObject {
     /**


[14/42] incubator-ignite git commit: # ignite-883

Posted by ag...@apache.org.
# ignite-883


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

Branch: refs/heads/ignite-389
Commit: ad2f4efce373b6fa08953ad030862175527cd009
Parents: 6bfc78e
Author: sboikov <sb...@gridgain.com>
Authored: Thu Jun 4 13:34:57 2015 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Thu Jun 4 13:34:57 2015 +0300

----------------------------------------------------------------------
 .../GridCachePartitionExchangeManager.java      | 26 +++++-----
 .../dht/atomic/GridDhtAtomicCache.java          | 18 ++++---
 .../GridDhtPartitionsExchangeFuture.java        | 50 +++++++++++---------
 3 files changed, 53 insertions(+), 41 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/ad2f4efc/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCachePartitionExchangeManager.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCachePartitionExchangeManager.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCachePartitionExchangeManager.java
index 488227b..3236bb5 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCachePartitionExchangeManager.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCachePartitionExchangeManager.java
@@ -1144,20 +1144,24 @@ public class GridCachePartitionExchangeManager<K, V> extends GridCacheSharedMana
 
         /** {@inheritDoc} */
         @Override public void onTimeout() {
-            if (!busyLock.readLock().tryLock())
-                return;
+            cctx.kernalContext().closure().runLocalSafe(new Runnable() {
+                @Override public void run() {
+                    if (!busyLock.readLock().tryLock())
+                        return;
 
-            try {
-                if (started.compareAndSet(false, true))
-                    refreshPartitions();
-            }
-            finally {
-                busyLock.readLock().unlock();
+                    try {
+                        if (started.compareAndSet(false, true))
+                            refreshPartitions();
+                    }
+                    finally {
+                        busyLock.readLock().unlock();
 
-                cctx.time().removeTimeoutObject(this);
+                        cctx.time().removeTimeoutObject(ResendTimeoutObject.this);
 
-                pendingResend.compareAndSet(this, null);
-            }
+                        pendingResend.compareAndSet(ResendTimeoutObject.this, null);
+                    }
+                }
+            });
         }
 
         /**

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/ad2f4efc/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 9ca80f9..ce91c6e 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
@@ -2775,14 +2775,18 @@ public class GridDhtAtomicCache<K, V> extends GridDhtCacheAdapter<K, V> {
         /** {@inheritDoc} */
         @Override public void onTimeout() {
             if (guard.compareAndSet(false, true)) {
-                writeLock().lock();
+                ctx.closures().runLocalSafe(new Runnable() {
+                    @Override public void run() {
+                        writeLock().lock();
 
-                try {
-                    finish();
-                }
-                finally {
-                    writeLock().unlock();
-                }
+                        try {
+                            finish();
+                        }
+                        finally {
+                            writeLock().unlock();
+                        }
+                    }
+                });
             }
         }
 

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/ad2f4efc/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/preloader/GridDhtPartitionsExchangeFuture.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/preloader/GridDhtPartitionsExchangeFuture.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/preloader/GridDhtPartitionsExchangeFuture.java
index e0bfee6..3362265 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/preloader/GridDhtPartitionsExchangeFuture.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/preloader/GridDhtPartitionsExchangeFuture.java
@@ -1389,30 +1389,34 @@ public class GridDhtPartitionsExchangeFuture extends GridFutureAdapter<AffinityT
             GridTimeoutObject timeoutObj = new GridTimeoutObjectAdapter(
                 cctx.gridConfig().getNetworkTimeout() * Math.max(1, cctx.gridConfig().getCacheConfiguration().length)) {
                 @Override public void onTimeout() {
-                    if (isDone())
-                        return;
+                    cctx.kernalContext().closure().runLocalSafe(new Runnable() {
+                        @Override public void run() {
+                            if (isDone())
+                                return;
+
+                            if (!enterBusy())
+                                return;
+
+                            try {
+                                U.warn(log,
+                                    "Retrying preload partition exchange due to timeout [done=" + isDone() +
+                                        ", dummy=" + dummy + ", exchId=" + exchId + ", rcvdIds=" + F.id8s(rcvdIds) +
+                                        ", rmtIds=" + F.id8s(rmtIds) + ", remaining=" + F.id8s(remaining()) +
+                                        ", init=" + init + ", initFut=" + initFut.isDone() +
+                                        ", ready=" + ready + ", replied=" + replied + ", added=" + added +
+                                        ", oldest=" + U.id8(oldestNode.get().id()) + ", oldestOrder=" +
+                                        oldestNode.get().order() + ", evtLatch=" + evtLatch.getCount() +
+                                        ", locNodeOrder=" + cctx.localNode().order() +
+                                        ", locNodeId=" + cctx.localNode().id() + ']',
+                                    "Retrying preload partition exchange due to timeout.");
 
-                    if (!enterBusy())
-                        return;
-
-                    try {
-                        U.warn(log,
-                            "Retrying preload partition exchange due to timeout [done=" + isDone() +
-                                ", dummy=" + dummy + ", exchId=" + exchId + ", rcvdIds=" + F.id8s(rcvdIds) +
-                                ", rmtIds=" + F.id8s(rmtIds) + ", remaining=" + F.id8s(remaining()) +
-                                ", init=" + init + ", initFut=" + initFut.isDone() +
-                                ", ready=" + ready + ", replied=" + replied + ", added=" + added +
-                                ", oldest=" + U.id8(oldestNode.get().id()) + ", oldestOrder=" +
-                                oldestNode.get().order() + ", evtLatch=" + evtLatch.getCount() +
-                                ", locNodeOrder=" + cctx.localNode().order() +
-                                ", locNodeId=" + cctx.localNode().id() + ']',
-                            "Retrying preload partition exchange due to timeout.");
-
-                        recheck();
-                    }
-                    finally {
-                        leaveBusy();
-                    }
+                                recheck();
+                            }
+                            finally {
+                                leaveBusy();
+                            }
+                        }
+                    });
                 }
             };
 


[38/42] incubator-ignite git commit: Merge remote-tracking branch 'origin/ignite-sprint-5' into ignite-sprint-5

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


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

Branch: refs/heads/ignite-389
Commit: 2a8e2abe4081334802f9950757b7d3776a3a6179
Parents: 837462f 015afdb
Author: sboikov <sb...@gridgain.com>
Authored: Mon Jun 8 14:55:48 2015 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Mon Jun 8 14:55:48 2015 +0300

----------------------------------------------------------------------
 .../apache/ignite/internal/IgniteKernal.java    |  26 +-
 .../org/apache/ignite/internal/IgnitionEx.java  |   8 +-
 .../internal/MarshallerContextAdapter.java      |  36 +-
 .../internal/managers/GridManagerAdapter.java   |   9 +
 .../checkpoint/GridCheckpointManager.java       |  52 +-
 .../discovery/GridDiscoveryManager.java         |  28 +-
 .../processors/cache/GridCacheAdapter.java      |   4 +
 .../GridCachePartitionExchangeManager.java      |  26 +-
 .../processors/cache/GridCacheTtlManager.java   |   9 +-
 .../dht/atomic/GridDhtAtomicCache.java          |  18 +-
 .../GridDhtPartitionsExchangeFuture.java        |  50 +-
 .../cache/transactions/IgniteTxManager.java     |   3 -
 .../datastructures/DataStructuresProcessor.java | 107 +++-
 .../service/GridServiceProcessor.java           |   4 +-
 .../timeout/GridSpiTimeoutObject.java           |  73 +++
 .../timeout/GridTimeoutProcessor.java           | 105 +++-
 .../util/nio/GridCommunicationClient.java       |  30 +-
 .../util/nio/GridNioRecoveryDescriptor.java     |  13 +-
 .../util/nio/GridTcpCommunicationClient.java    | 554 -------------------
 .../util/nio/GridTcpNioCommunicationClient.java |   8 -
 .../org/apache/ignite/spi/IgniteSpiAdapter.java |  27 +-
 .../org/apache/ignite/spi/IgniteSpiContext.java |  10 +
 .../ignite/spi/IgniteSpiTimeoutObject.java      |  44 ++
 .../spi/checkpoint/noop/NoopCheckpointSpi.java  |   3 +-
 .../communication/tcp/TcpCommunicationSpi.java  | 438 ++++-----------
 .../tcp/TcpCommunicationSpiMBean.java           |   2 -
 .../ignite/spi/discovery/tcp/ClientImpl.java    |   3 -
 .../ignite/spi/discovery/tcp/ServerImpl.java    |  10 +-
 .../spi/discovery/tcp/TcpDiscoverySpi.java      | 156 +-----
 .../IgniteCountDownLatchAbstractSelfTest.java   | 102 ++++
 .../IgniteCacheClientNearCacheExpiryTest.java   | 103 ++++
 .../IgniteCacheExpiryPolicyTestSuite.java       |   2 +
 .../internal/util/nio/GridNioSelfTest.java      |   2 +-
 .../GridTcpCommunicationSpiAbstractTest.java    |   4 +-
 ...mmunicationSpiConcurrentConnectSelfTest.java |   2 +-
 .../GridTcpCommunicationSpiConfigSelfTest.java  |   2 -
 ...cpCommunicationSpiMultithreadedSelfTest.java |   2 +-
 .../discovery/AbstractDiscoverySelfTest.java    |  13 +-
 .../tcp/TcpClientDiscoverySpiSelfTest.java      |  25 +
 .../testframework/GridSpiTestContext.java       |  10 +
 40 files changed, 920 insertions(+), 1203 deletions(-)
----------------------------------------------------------------------



[16/42] incubator-ignite git commit: Merge branches 'ignite-941' and 'ignite-sprint-5' of https://git-wip-us.apache.org/repos/asf/incubator-ignite into ignite-941

Posted by ag...@apache.org.
Merge branches 'ignite-941' and 'ignite-sprint-5' of https://git-wip-us.apache.org/repos/asf/incubator-ignite into ignite-941


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

Branch: refs/heads/ignite-389
Commit: b58c892e149567b3ca8d6746cb87cc11a4808dec
Parents: e9cea3a c9f7291
Author: Valentin Kulichenko <vk...@gridgain.com>
Authored: Thu Jun 4 16:28:19 2015 -0700
Committer: Valentin Kulichenko <vk...@gridgain.com>
Committed: Thu Jun 4 16:28:19 2015 -0700

----------------------------------------------------------------------
 .../configuration/CacheConfiguration.java       |  13 +-
 .../apache/ignite/internal/IgniteKernal.java    |   3 +
 .../managers/communication/GridIoManager.java   | 117 ++++----
 .../dht/GridClientPartitionTopology.java        |   2 +-
 .../dht/GridDhtPartitionTopologyImpl.java       |  16 +-
 .../GridDhtPartitionsExchangeFuture.java        |  29 +-
 .../dht/preloader/GridDhtPreloader.java         |   2 +-
 .../processors/hadoop/HadoopTaskContext.java    |  14 +-
 .../igfs/IgfsSecondaryFileSystemImpl.java       |   2 +-
 .../internal/visor/query/VisorQueryJob.java     |   2 +-
 .../ignite/spi/discovery/tcp/ClientImpl.java    |   3 +
 .../ignite/spi/discovery/tcp/ServerImpl.java    |  31 --
 .../spi/discovery/tcp/TcpDiscoveryImpl.java     |  56 +++-
 .../tcp/ipfinder/TcpDiscoveryIpFinder.java      |  10 +-
 .../TcpDiscoveryMulticastIpFinder.java          |  47 +++-
 .../cache/IgniteDynamicCacheStartSelfTest.java  |  62 ++++
 ...niteDynamicCacheWithConfigStartSelfTest.java |  97 +++++++
 .../igfs/IgfsClientCacheSelfTest.java           |   9 +-
 .../IgniteMessagingWithClientTest.java          | 164 +++++++++++
 .../tcp/TcpClientDiscoverySpiMulticastTest.java | 129 +++++++++
 .../ignite/testsuites/IgniteBasicTestSuite.java |   1 +
 .../testsuites/IgniteCacheTestSuite4.java       |   1 +
 .../IgniteSpiDiscoverySelfTestSuite.java        |   1 +
 .../gce/TcpDiscoveryGoogleStorageIpFinder.java  |  43 +--
 .../fs/IgniteHadoopFileSystemCounterWriter.java |  14 +-
 .../hadoop/fs/v1/IgniteHadoopFileSystem.java    |  70 ++---
 .../hadoop/fs/v2/IgniteHadoopFileSystem.java    |   2 +-
 .../processors/hadoop/HadoopDefaultJobInfo.java |   2 +-
 .../internal/processors/hadoop/HadoopUtils.java | 282 ++++++++++++++++++-
 .../hadoop/SecondaryFileSystemProvider.java     |   4 +-
 .../hadoop/taskexecutor/HadoopRunnableTask.java |  20 +-
 .../processors/hadoop/v2/HadoopV2Job.java       |  31 +-
 .../hadoop/v2/HadoopV2JobResourceManager.java   |  26 +-
 .../hadoop/v2/HadoopV2TaskContext.java          |  48 +++-
 .../hadoop/HadoopClientProtocolSelfTest.java    |   6 +-
 .../hadoop/HadoopAbstractSelfTest.java          |  14 +-
 .../hadoop/HadoopCommandLineTest.java           |  14 +-
 .../processors/hadoop/HadoopMapReduceTest.java  | 176 +++++++++++-
 .../hadoop/HadoopTaskExecutionSelfTest.java     |   2 +-
 .../hadoop/HadoopTasksAllVersionsTest.java      |  15 +-
 .../processors/hadoop/HadoopTasksV1Test.java    |   5 +-
 .../processors/hadoop/HadoopTasksV2Test.java    |   5 +-
 .../processors/hadoop/HadoopV2JobSelfTest.java  |   6 +-
 .../collections/HadoopAbstractMapTest.java      |  12 +
 ...acheConfigurationPrimitiveTypesSelfTest.java | 104 +++++++
 .../IgniteCacheWithIndexingTestSuite.java       |   2 +
 .../commands/cache/VisorCacheScanCommand.scala  |   2 +-
 47 files changed, 1443 insertions(+), 273 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/b58c892e/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteCacheTestSuite4.java
----------------------------------------------------------------------