You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@ignite.apache.org by sb...@apache.org on 2017/10/04 08:45:03 UTC

[20/25] ignite git commit: IGNITE-6285 Enhance persistent store paths handling - Fixes #2775.

IGNITE-6285 Enhance persistent store paths handling - Fixes #2775.

Signed-off-by: Alexey Goncharuk <al...@gmail.com>


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

Branch: refs/heads/ignite-5937
Commit: 62f3c4c52f5aad82e51e93b7981e670090cddd4c
Parents: 855fe4b
Author: dpavlov <dp...@gridgain.com>
Authored: Wed Oct 4 09:46:16 2017 +0300
Committer: Alexey Goncharuk <al...@gmail.com>
Committed: Wed Oct 4 09:46:16 2017 +0300

----------------------------------------------------------------------
 .../FoldersReuseCompatibilityTest.java          | 224 ++++++
 ...itePersistenceCompatibilityAbstractTest.java |   3 +
 .../IgniteCompatibilityBasicTestSuite.java      |   3 +
 .../apache/ignite/IgniteSystemProperties.java   |   7 +
 .../ignite/internal/GridKernalContext.java      |   6 +
 .../ignite/internal/GridKernalContextImpl.java  |  17 +-
 .../apache/ignite/internal/IgniteKernal.java    |   4 +-
 .../discovery/GridDiscoveryManager.java         |  55 +-
 .../cache/binary/BinaryMetadataFileStore.java   |   6 +-
 .../GridCacheDatabaseSharedManager.java         |  91 ++-
 .../IgniteCacheDatabaseSharedManager.java       |  13 +-
 .../persistence/file/FilePageStoreManager.java  |  29 +-
 .../filename/PdsConsistentIdProcessor.java      | 568 +++++++++++++++
 .../persistence/filename/PdsFolderSettings.java | 138 ++++
 .../filename/PdsFoldersResolver.java            |  33 +
 .../wal/FileWriteAheadLogManager.java           |  20 +-
 .../wal/reader/StandaloneGridKernalContext.java |  32 +-
 .../spi/discovery/tcp/TcpDiscoverySpi.java      |   4 +-
 .../tcp/internal/TcpDiscoveryNode.java          |  15 +
 .../IgniteUidAsConsistentIdMigrationTest.java   | 712 +++++++++++++++++++
 .../db/wal/IgniteWalRecoveryTest.java           |  13 +-
 .../db/wal/reader/IgniteWalReaderTest.java      | 187 +++--
 .../db/wal/reader/MockWalIteratorFactory.java   |  25 +-
 .../ignite/testsuites/IgnitePdsTestSuite2.java  |   4 +
 24 files changed, 1995 insertions(+), 214 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/ignite/blob/62f3c4c5/modules/compatibility/src/test/java/org/apache/ignite/compatibility/persistence/FoldersReuseCompatibilityTest.java
----------------------------------------------------------------------
diff --git a/modules/compatibility/src/test/java/org/apache/ignite/compatibility/persistence/FoldersReuseCompatibilityTest.java b/modules/compatibility/src/test/java/org/apache/ignite/compatibility/persistence/FoldersReuseCompatibilityTest.java
new file mode 100644
index 0000000..1775013
--- /dev/null
+++ b/modules/compatibility/src/test/java/org/apache/ignite/compatibility/persistence/FoldersReuseCompatibilityTest.java
@@ -0,0 +1,224 @@
+/*
+ * 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.compatibility.persistence;
+
+import java.io.File;
+import java.io.IOException;
+import java.util.Arrays;
+import java.util.Set;
+import java.util.TreeSet;
+import org.apache.ignite.Ignite;
+import org.apache.ignite.IgniteCheckedException;
+import org.apache.ignite.configuration.IgniteConfiguration;
+import org.apache.ignite.configuration.MemoryConfiguration;
+import org.apache.ignite.configuration.MemoryPolicyConfiguration;
+import org.apache.ignite.configuration.PersistentStoreConfiguration;
+import org.apache.ignite.internal.IgniteEx;
+import org.apache.ignite.internal.processors.cache.GridCacheAbstractFullApiSelfTest;
+import org.apache.ignite.internal.processors.cache.persistence.filename.PdsConsistentIdProcessor;
+import org.apache.ignite.internal.util.typedef.internal.U;
+import org.apache.ignite.lang.IgniteInClosure;
+import org.apache.ignite.spi.discovery.tcp.TcpDiscoverySpi;
+import org.jetbrains.annotations.NotNull;
+
+import static org.apache.ignite.internal.processors.cache.persistence.filename.PdsConsistentIdProcessor.parseSubFolderName;
+
+/**
+ * Test for new and old style persistent storage folders generation and compatible startup of current ignite version
+ */
+public class FoldersReuseCompatibilityTest extends IgnitePersistenceCompatibilityAbstractTest {
+    /** Cache name for test. */
+    private static final String CACHE_NAME = "dummy";
+
+    /** Key to store in previous version of ignite */
+    private static final String KEY = "ObjectFromPast";
+
+    /** Value to store in previous version of ignite */
+    private static final String VAL = "ValueFromPast";
+
+    /** {@inheritDoc} */
+    @Override protected void afterTest() throws Exception {
+        // No-op. super.afterTest();
+        stopAllGrids();
+    }
+
+    /** {@inheritDoc} */
+    @Override protected IgniteConfiguration getConfiguration(String igniteInstanceName) throws Exception {
+        final IgniteConfiguration cfg = super.getConfiguration(igniteInstanceName);
+
+        configPersistence(cfg);
+
+        return cfg;
+    }
+
+    /**
+     * Test startup of current ignite version using DB storage folder from previous version of Ignite. Expected to start
+     * successfully with existing DB
+     *
+     * @throws Exception if failed.
+     */
+    public void testFoldersReuseCompatibility_2_2() throws Exception {
+        runFoldersReuse("2.2.0");
+    }
+
+    /**
+     * Test startup of current ignite version using DB storage folder from previous version of Ignite. Expected to start
+     * successfully with existing DB
+     *
+     * @throws Exception if failed.
+     */
+    public void testFoldersReuseCompatibility_2_1() throws Exception {
+        runFoldersReuse("2.1.0");
+    }
+
+    /**
+     * Test startup of current ignite version using DB storage folder from previous version of Ignite. Expected to start
+     * successfully with existing DB
+     *
+     * @param ver 3 digit Ignite version to check compatibility with
+     * @throws Exception if failed.
+     */
+    private void runFoldersReuse(String ver) throws Exception {
+        final IgniteEx grid = startGrid(1, ver, new ConfigurationClosure(), new PostStartupClosure());
+
+        grid.close();
+        stopAllGrids();
+
+        IgniteEx ignite = startGrid(0);
+
+        ignite.active(true);
+        ignite.getOrCreateCache("cache2createdForNewGrid").put("Object", "Value");
+        assertEquals(1, ignite.context().discovery().topologyVersion());
+
+        assertPdsDirsDefaultExist(U.maskForFileName(ignite.cluster().node().consistentId().toString()));
+
+        assertEquals(VAL, ignite.cache(CACHE_NAME).get(KEY));
+
+        assertNodeIndexesInFolder();// should not create any new style directories
+
+        stopAllGrids();
+    }
+
+    /** Started node test actions closure. */
+    private static class PostStartupClosure implements IgniteInClosure<Ignite> {
+        /** {@inheritDoc} */
+        @Override public void apply(Ignite ignite) {
+            ignite.active(true);
+            ignite.getOrCreateCache(CACHE_NAME).put(KEY, VAL);
+        }
+    }
+
+    /** Setup compatible node closure. */
+    private static class ConfigurationClosure implements IgniteInClosure<IgniteConfiguration> {
+        /** {@inheritDoc} */
+        @Override public void apply(IgniteConfiguration cfg) {
+            cfg.setLocalHost("127.0.0.1");
+            TcpDiscoverySpi disco = new TcpDiscoverySpi();
+            disco.setIpFinder(GridCacheAbstractFullApiSelfTest.LOCAL_IP_FINDER);
+
+            cfg.setDiscoverySpi(disco);
+
+            configPersistence(cfg);
+        }
+    }
+
+    /**
+     * Setup persistence for compatible and current version node.
+     *
+     * @param cfg ignite config to setup.
+     */
+    private static void configPersistence(IgniteConfiguration cfg) {
+        final PersistentStoreConfiguration psCfg = new PersistentStoreConfiguration();
+
+        cfg.setPersistentStoreConfiguration(psCfg);
+
+        final MemoryConfiguration memCfg = new MemoryConfiguration();
+        final MemoryPolicyConfiguration memPolCfg = new MemoryPolicyConfiguration();
+
+        memPolCfg.setMaxSize(32 * 1024 * 1024); // we don't need much memory for this test
+        memCfg.setMemoryPolicies(memPolCfg);
+        cfg.setMemoryConfiguration(memCfg);
+    }
+
+    /**
+     * @param indexes expected new style node indexes in folders
+     * @throws IgniteCheckedException if failed
+     */
+    private void assertNodeIndexesInFolder(Integer... indexes) throws IgniteCheckedException {
+        assertEquals(new TreeSet<>(Arrays.asList(indexes)), getAllNodeIndexesInFolder());
+    }
+
+    /**
+     * @return set of all indexes of nodes found in work folder
+     * @throws IgniteCheckedException if failed.
+     */
+    @NotNull private Set<Integer> getAllNodeIndexesInFolder() throws IgniteCheckedException {
+        final File curFolder = new File(U.defaultWorkDirectory(), PdsConsistentIdProcessor.DB_DEFAULT_FOLDER);
+        final Set<Integer> indexes = new TreeSet<>();
+        final File[] files = curFolder.listFiles(PdsConsistentIdProcessor.DB_SUBFOLDERS_NEW_STYLE_FILTER);
+
+        for (File file : files) {
+            final PdsConsistentIdProcessor.FolderCandidate uid
+                = parseSubFolderName(file, log);
+
+            if (uid != null)
+                indexes.add(uid.nodeIndex());
+        }
+
+        return indexes;
+    }
+
+    /**
+     * Checks existence of all storage-related directories
+     *
+     * @param subDirName sub directories name expected
+     * @throws IgniteCheckedException if IO error occur
+     */
+    private void assertPdsDirsDefaultExist(String subDirName) throws IgniteCheckedException {
+        assertDirectoryExist("binary_meta", subDirName);
+        assertDirectoryExist(PersistentStoreConfiguration.DFLT_WAL_STORE_PATH, subDirName);
+        assertDirectoryExist(PersistentStoreConfiguration.DFLT_WAL_ARCHIVE_PATH, subDirName);
+        assertDirectoryExist(PdsConsistentIdProcessor.DB_DEFAULT_FOLDER, subDirName);
+    }
+
+    /**
+     * Checks one folder existence
+     *
+     * @param subFolderNames subfolders array to touch
+     * @throws IgniteCheckedException if IO error occur
+     */
+    private void assertDirectoryExist(String... subFolderNames) throws IgniteCheckedException {
+        File curFolder = new File(U.defaultWorkDirectory());
+
+        for (String name : subFolderNames) {
+            curFolder = new File(curFolder, name);
+        }
+
+        final String path;
+        try {
+            path = curFolder.getCanonicalPath();
+        }
+        catch (IOException e) {
+            throw new IgniteCheckedException("Failed to convert path: [" + curFolder.getAbsolutePath() + "]", e);
+        }
+
+        assertTrue("Directory " + Arrays.asList(subFolderNames).toString()
+            + " is expected to exist [" + path + "]", curFolder.exists() && curFolder.isDirectory());
+    }
+
+}

http://git-wip-us.apache.org/repos/asf/ignite/blob/62f3c4c5/modules/compatibility/src/test/java/org/apache/ignite/compatibility/persistence/IgnitePersistenceCompatibilityAbstractTest.java
----------------------------------------------------------------------
diff --git a/modules/compatibility/src/test/java/org/apache/ignite/compatibility/persistence/IgnitePersistenceCompatibilityAbstractTest.java b/modules/compatibility/src/test/java/org/apache/ignite/compatibility/persistence/IgnitePersistenceCompatibilityAbstractTest.java
index d76b862..f39b6f6 100644
--- a/modules/compatibility/src/test/java/org/apache/ignite/compatibility/persistence/IgnitePersistenceCompatibilityAbstractTest.java
+++ b/modules/compatibility/src/test/java/org/apache/ignite/compatibility/persistence/IgnitePersistenceCompatibilityAbstractTest.java
@@ -45,6 +45,9 @@ public abstract class IgnitePersistenceCompatibilityAbstractTest extends IgniteC
     @Override protected void afterTest() throws Exception {
         super.afterTest();
 
+        //protection if test failed to finish, e.g. by error
+        stopAllGrids();
+
         assert deleteDefaultDBWorkDirectory() : "Couldn't delete DB work directory.";
     }
 

http://git-wip-us.apache.org/repos/asf/ignite/blob/62f3c4c5/modules/compatibility/src/test/java/org/apache/ignite/compatibility/testsuites/IgniteCompatibilityBasicTestSuite.java
----------------------------------------------------------------------
diff --git a/modules/compatibility/src/test/java/org/apache/ignite/compatibility/testsuites/IgniteCompatibilityBasicTestSuite.java b/modules/compatibility/src/test/java/org/apache/ignite/compatibility/testsuites/IgniteCompatibilityBasicTestSuite.java
index b54b396..351a0e7 100644
--- a/modules/compatibility/src/test/java/org/apache/ignite/compatibility/testsuites/IgniteCompatibilityBasicTestSuite.java
+++ b/modules/compatibility/src/test/java/org/apache/ignite/compatibility/testsuites/IgniteCompatibilityBasicTestSuite.java
@@ -19,6 +19,7 @@ package org.apache.ignite.compatibility.testsuites;
 
 import junit.framework.TestSuite;
 import org.apache.ignite.compatibility.persistence.DummyPersistenceCompatibilityTest;
+import org.apache.ignite.compatibility.persistence.FoldersReuseCompatibilityTest;
 
 /**
  * Compatibility tests basic test suite.
@@ -33,6 +34,8 @@ public class IgniteCompatibilityBasicTestSuite {
 
         suite.addTestSuite(DummyPersistenceCompatibilityTest.class);
 
+        suite.addTestSuite(FoldersReuseCompatibilityTest.class);
+
         return suite;
     }
 }

http://git-wip-us.apache.org/repos/asf/ignite/blob/62f3c4c5/modules/core/src/main/java/org/apache/ignite/IgniteSystemProperties.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/IgniteSystemProperties.java b/modules/core/src/main/java/org/apache/ignite/IgniteSystemProperties.java
index e1e72f7..d7b4de9 100644
--- a/modules/core/src/main/java/org/apache/ignite/IgniteSystemProperties.java
+++ b/modules/core/src/main/java/org/apache/ignite/IgniteSystemProperties.java
@@ -737,6 +737,13 @@ public final class IgniteSystemProperties {
     public static final String IGNITE_WAL_SERIALIZER_VERSION = "IGNITE_WAL_SERIALIZER_VERSION";
 
     /**
+     * When set to {@code true}, Data store folders are generated only by consistent id, and no consistent ID will be
+     * set based on existing data store folders. This option also enables compatible folder generation mode as it was
+     * before 2.3.
+     */
+    public static final String IGNITE_DATA_STORAGE_FOLDER_BY_CONSISTENT_ID = "IGNITE_DATA_STORAGE_FOLDER_BY_CONSISTENT_ID";
+
+    /**
      * Enforces singleton.
      */
     private IgniteSystemProperties() {

http://git-wip-us.apache.org/repos/asf/ignite/blob/62f3c4c5/modules/core/src/main/java/org/apache/ignite/internal/GridKernalContext.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/GridKernalContext.java b/modules/core/src/main/java/org/apache/ignite/internal/GridKernalContext.java
index 99c7cce..210b401 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/GridKernalContext.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/GridKernalContext.java
@@ -34,6 +34,7 @@ import org.apache.ignite.internal.managers.indexing.GridIndexingManager;
 import org.apache.ignite.internal.managers.loadbalancer.GridLoadBalancerManager;
 import org.apache.ignite.internal.processors.affinity.GridAffinityProcessor;
 import org.apache.ignite.internal.processors.cache.GridCacheProcessor;
+import org.apache.ignite.internal.processors.cache.persistence.filename.PdsFoldersResolver;
 import org.apache.ignite.internal.processors.cacheobject.IgniteCacheObjectProcessor;
 import org.apache.ignite.internal.processors.closure.GridClosureProcessor;
 import org.apache.ignite.internal.processors.cluster.ClusterProcessor;
@@ -643,4 +644,9 @@ public interface GridKernalContext extends Iterable<GridComponent> {
      * @return Platform processor.
      */
     public PlatformProcessor platform();
+
+    /**
+     * @return PDS mode folder name resolver, also generates consistent ID in case new folder naming is used
+     */
+    public PdsFoldersResolver pdsFolderResolver();
 }

http://git-wip-us.apache.org/repos/asf/ignite/blob/62f3c4c5/modules/core/src/main/java/org/apache/ignite/internal/GridKernalContextImpl.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/GridKernalContextImpl.java b/modules/core/src/main/java/org/apache/ignite/internal/GridKernalContextImpl.java
index 07e5970..1f0292c 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/GridKernalContextImpl.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/GridKernalContextImpl.java
@@ -49,6 +49,7 @@ import org.apache.ignite.internal.processors.affinity.GridAffinityProcessor;
 import org.apache.ignite.internal.processors.cache.CacheConflictResolutionManager;
 import org.apache.ignite.internal.processors.cache.GridCacheProcessor;
 import org.apache.ignite.internal.processors.cache.binary.CacheObjectBinaryProcessorImpl;
+import org.apache.ignite.internal.processors.cache.persistence.filename.PdsFoldersResolver;
 import org.apache.ignite.internal.processors.cacheobject.IgniteCacheObjectProcessor;
 import org.apache.ignite.internal.processors.closure.GridClosureProcessor;
 import org.apache.ignite.internal.processors.cluster.ClusterProcessor;
@@ -377,6 +378,9 @@ public class GridKernalContextImpl implements GridKernalContext, Externalizable
     /** */
     private volatile boolean disconnected;
 
+    /** PDS mode folder name resolver, also generates consistent ID in case new folder naming is used */
+    private PdsFoldersResolver pdsFolderRslvr;
+
     /**
      * No-arg constructor is required by externalization.
      */
@@ -536,7 +540,7 @@ public class GridKernalContextImpl implements GridKernalContext, Externalizable
         else if (comp instanceof GridCacheProcessor)
             cacheProc = (GridCacheProcessor)comp;
         else if (comp instanceof GridClusterStateProcessor)
-            stateProc =(GridClusterStateProcessor)comp;
+            stateProc = (GridClusterStateProcessor)comp;
         else if (comp instanceof GridTaskSessionProcessor)
             sesProc = (GridTaskSessionProcessor)comp;
         else if (comp instanceof GridPortProcessor)
@@ -576,9 +580,11 @@ public class GridKernalContextImpl implements GridKernalContext, Externalizable
         else if (comp instanceof PlatformProcessor)
             platformProc = (PlatformProcessor)comp;
         else if (comp instanceof PoolProcessor)
-            poolProc = (PoolProcessor) comp;
+            poolProc = (PoolProcessor)comp;
         else if (comp instanceof GridMarshallerMappingProcessor)
             mappingProc = (GridMarshallerMappingProcessor)comp;
+        else if (comp instanceof PdsFoldersResolver)
+            pdsFolderRslvr = (PdsFoldersResolver)comp;
         else if (!(comp instanceof DiscoveryNodeValidationProcessor
                 || comp instanceof PlatformPluginProcessor))
             assert (comp instanceof GridPluginComponent) : "Unknown manager class: " + comp.getClass();
@@ -1005,7 +1011,7 @@ public class GridKernalContextImpl implements GridKernalContext, Externalizable
     }
 
     /** {@inheritDoc} */
-    public Map<String, ? extends ExecutorService> customExecutors() {
+    @Override public Map<String, ? extends ExecutorService> customExecutors() {
         return customExecSvcs;
     }
 
@@ -1069,6 +1075,11 @@ public class GridKernalContextImpl implements GridKernalContext, Externalizable
         this.disconnected = disconnected;
     }
 
+    /**{@inheritDoc}*/
+    @Override public PdsFoldersResolver pdsFolderResolver() {
+        return pdsFolderRslvr;
+    }
+
     /** {@inheritDoc} */
     @Override public String toString() {
         return S.toString(GridKernalContextImpl.class, this);

http://git-wip-us.apache.org/repos/asf/ignite/blob/62f3c4c5/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 f5d736a..759bf64 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
@@ -115,6 +115,7 @@ import org.apache.ignite.internal.processors.cache.IgniteCacheProxy;
 import org.apache.ignite.internal.processors.cache.IgniteInternalCache;
 import org.apache.ignite.internal.processors.cache.binary.CacheObjectBinaryProcessorImpl;
 import org.apache.ignite.internal.processors.cache.persistence.MemoryPolicy;
+import org.apache.ignite.internal.processors.cache.persistence.filename.PdsConsistentIdProcessor;
 import org.apache.ignite.internal.processors.cacheobject.IgniteCacheObjectProcessor;
 import org.apache.ignite.internal.processors.closure.GridClosureProcessor;
 import org.apache.ignite.internal.processors.cluster.ClusterProcessor;
@@ -930,13 +931,14 @@ public class IgniteKernal implements IgniteEx, IgniteMXBean, Externalizable {
 
             // Assign discovery manager to context before other processors start so they
             // are able to register custom event listener.
-            GridManager discoMgr = new GridDiscoveryManager(ctx);
+            final GridManager discoMgr = new GridDiscoveryManager(ctx);
 
             ctx.add(discoMgr, false);
 
             // Start processors before discovery manager, so they will
             // be able to start receiving messages once discovery completes.
             try {
+                startProcessor(new PdsConsistentIdProcessor(ctx));
                 startProcessor(createComponent(DiscoveryNodeValidationProcessor.class, ctx));
                 startProcessor(new  GridAffinityProcessor(ctx));
                 startProcessor(createComponent(GridSegmentationProcessor.class, ctx));

http://git-wip-us.apache.org/repos/asf/ignite/blob/62f3c4c5/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 527399d..14485d2 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
@@ -17,6 +17,7 @@
 
 package org.apache.ignite.internal.managers.discovery;
 
+import java.io.Serializable;
 import java.lang.management.GarbageCollectorMXBean;
 import java.lang.management.ManagementFactory;
 import java.lang.management.MemoryMXBean;
@@ -116,6 +117,7 @@ import org.apache.ignite.spi.discovery.DiscoverySpiListener;
 import org.apache.ignite.spi.discovery.DiscoverySpiNodeAuthenticator;
 import org.apache.ignite.spi.discovery.DiscoverySpiOrderSupport;
 import org.apache.ignite.spi.discovery.tcp.TcpDiscoverySpi;
+import org.apache.ignite.spi.discovery.tcp.internal.TcpDiscoveryNode;
 import org.apache.ignite.thread.IgniteThread;
 import org.jetbrains.annotations.NotNull;
 import org.jetbrains.annotations.Nullable;
@@ -281,12 +283,12 @@ public class GridDiscoveryManager extends GridManagerAdapter<DiscoverySpi> {
     /** */
     private final CountDownLatch startLatch = new CountDownLatch(1);
 
-    /** */
-    private Object consistentId;
-
     /** Discovery spi registered flag. */
     private boolean registeredDiscoSpi;
 
+    /** Local node compatibility consistent ID. */
+    private Serializable consistentId;
+
     /** @param ctx Context. */
     public GridDiscoveryManager(GridKernalContext ctx) {
         super(ctx, ctx.config().getDiscoverySpi());
@@ -549,6 +551,13 @@ public class GridDiscoveryManager extends GridManagerAdapter<DiscoverySpi> {
             @Override public void onLocalNodeInitialized(ClusterNode locNode) {
                 for (IgniteInClosure<ClusterNode> lsnr : locNodeInitLsnrs)
                     lsnr.apply(locNode);
+
+                if (locNode instanceof TcpDiscoveryNode) {
+                    final TcpDiscoveryNode node = (TcpDiscoveryNode)locNode;
+
+                    if (consistentId != null)
+                        node.setConsistentId(consistentId);
+                }
             }
 
             @Override public void onDiscovery(
@@ -2017,22 +2026,42 @@ public class GridDiscoveryManager extends GridManagerAdapter<DiscoverySpi> {
 
     /**
      * @return Consistent ID.
+     * @deprecated Use PdsConsistentIdProcessor to get actual consistent ID
      */
-    public Object consistentId() {
-        if (consistentId == null) {
-            try {
-                inject();
-            }
-            catch (IgniteCheckedException e) {
-                throw new IgniteException("Failed to init consistent ID.", e);
-            }
+    @Deprecated
+    public Serializable consistentId() {
+        if (consistentId == null)
+            consistentId = getInjectedDiscoverySpi().consistentId();
+
+        return consistentId;
+    }
 
-            consistentId = getSpi().consistentId();
+    /**
+     * Performs injection of discovery SPI if needed, then provides DiscoverySpi SPI.
+     * Manual injection is required because normal startup of SPI is done after processor started.
+     *
+     * @return Wrapped DiscoverySpi SPI.
+     */
+    private DiscoverySpi getInjectedDiscoverySpi() {
+        try {
+            inject();
+        }
+        catch (IgniteCheckedException e) {
+            throw new IgniteException("Failed to init consistent ID.", e);
         }
+        return getSpi();
+    }
 
-        return consistentId;
+    /**
+     * Sets TCP local node consistent ID. This setter is to be called before node init in SPI
+     *
+     * @param consistentId New value of consistent ID to be used in local node initialization
+     */
+    public void consistentId(final Serializable consistentId) {
+        this.consistentId = consistentId;
     }
 
+
     /** @return Topology version. */
     public long topologyVersion() {
         return topSnap.get().topVer.topologyVersion();

http://git-wip-us.apache.org/repos/asf/ignite/blob/62f3c4c5/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/binary/BinaryMetadataFileStore.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/binary/BinaryMetadataFileStore.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/binary/BinaryMetadataFileStore.java
index 2d4114f..420cde5 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/binary/BinaryMetadataFileStore.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/binary/BinaryMetadataFileStore.java
@@ -34,7 +34,7 @@ import org.jetbrains.annotations.Nullable;
  * which may lead to segmentation of nodes from cluster.
  */
 class BinaryMetadataFileStore {
-    /** */
+    /** Link to resolved binary metadata directory. Null for non persistent mode */
     private File workDir;
 
     /** */
@@ -68,14 +68,14 @@ class BinaryMetadataFileStore {
         if (binaryMetadataFileStoreDir != null)
             workDir = binaryMetadataFileStoreDir;
         else {
-            String consId = U.maskForFileName(ctx.discovery().consistentId().toString());
+            final String subFolder = ctx.pdsFolderResolver().resolveFolders().folderName();
 
             workDir = new File(U.resolveWorkDirectory(
                 ctx.config().getWorkDirectory(),
                 "binary_meta",
                 false
             ),
-                consId);
+                subFolder);
         }
 
         U.ensureDirectory(workDir, "directory for serialized binary metadata", log);

http://git-wip-us.apache.org/repos/asf/ignite/blob/62f3c4c5/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/GridCacheDatabaseSharedManager.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/GridCacheDatabaseSharedManager.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/GridCacheDatabaseSharedManager.java
index 9f1ccb4..2d89942 100755
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/GridCacheDatabaseSharedManager.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/GridCacheDatabaseSharedManager.java
@@ -62,6 +62,7 @@ import org.apache.ignite.IgniteException;
 import org.apache.ignite.IgniteLogger;
 import org.apache.ignite.IgniteSystemProperties;
 import org.apache.ignite.PersistenceMetrics;
+import org.apache.ignite.cluster.ClusterNode;
 import org.apache.ignite.configuration.CheckpointWriteOrder;
 import org.apache.ignite.configuration.DataPageEvictionMode;
 import org.apache.ignite.configuration.IgniteConfiguration;
@@ -74,6 +75,7 @@ import org.apache.ignite.internal.GridKernalContext;
 import org.apache.ignite.internal.IgniteInternalFuture;
 import org.apache.ignite.internal.IgniteInterruptedCheckedException;
 import org.apache.ignite.internal.NodeStoppingException;
+import org.apache.ignite.internal.managers.discovery.GridDiscoveryManager;
 import org.apache.ignite.internal.mem.DirectMemoryProvider;
 import org.apache.ignite.internal.pagemem.FullPageId;
 import org.apache.ignite.internal.pagemem.PageIdUtils;
@@ -280,8 +282,11 @@ public class GridCacheDatabaseSharedManager extends IgniteCacheDatabaseSharedMan
     /** */
     private final ConcurrentMap<Integer, IgniteInternalFuture> idxRebuildFuts = new ConcurrentHashMap<>();
 
-    /** Lock holder. */
-    private FileLockHolder fileLockHolder;
+    /**
+     * Lock holder for compatible folders mode. Null if lock holder was created at start node. <br>
+     * In this case lock is held on PDS resover manager and it is not required to manage locking here
+     */
+    @Nullable private FileLockHolder fileLockHolder;
 
     /** Lock wait time. */
     private final long lockWaitTime;
@@ -367,7 +372,9 @@ public class GridCacheDatabaseSharedManager extends IgniteCacheDatabaseSharedMan
 
         snapshotMgr = cctx.snapshot();
 
-        if (!cctx.kernalContext().clientNode()) {
+        final GridKernalContext kernalCtx = cctx.kernalContext();
+
+        if (!kernalCtx.clientNode()) {
             IgnitePageStoreManager store = cctx.pageStore();
 
             assert store instanceof FilePageStoreManager : "Invalid page store manager was created: " + store;
@@ -379,7 +386,11 @@ public class GridCacheDatabaseSharedManager extends IgniteCacheDatabaseSharedMan
             if (!U.mkdirs(cpDir))
                 throw new IgniteCheckedException("Could not create directory for checkpoint metadata: " + cpDir);
 
-            fileLockHolder = new FileLockHolder(storeMgr.workDir().getPath(), cctx.kernalContext(), log);
+            final FileLockHolder preLocked = kernalCtx.pdsFolderResolver()
+                .resolveFolders()
+                .getLockedFileLockHolder();
+            if (preLocked == null)
+                fileLockHolder = new FileLockHolder(storeMgr.workDir().getPath(), kernalCtx, log);
 
             persStoreMetrics.wal(cctx.wal());
         }
@@ -488,8 +499,11 @@ public class GridCacheDatabaseSharedManager extends IgniteCacheDatabaseSharedMan
         /* Must be here, because after deactivate we can invoke activate and file lock must be already configured */
         stopping = false;
 
-        if (!cctx.localNode().isClient())
-            fileLockHolder = new FileLockHolder(storeMgr.workDir().getPath(), cctx.kernalContext(), log);
+        if (!cctx.localNode().isClient()) {
+            //we replace lock with new instance (only if we're responsible for locking folders)
+            if (fileLockHolder != null)
+                fileLockHolder = new FileLockHolder(storeMgr.workDir().getPath(), cctx.kernalContext(), log);
+        }
     }
 
     /**
@@ -592,20 +606,24 @@ public class GridCacheDatabaseSharedManager extends IgniteCacheDatabaseSharedMan
 
     /** {@inheritDoc} */
     @Override public void lock() throws IgniteCheckedException {
-        if (log.isDebugEnabled())
-            log.debug("Try to capture file lock [nodeId=" +
-                cctx.localNodeId() + " path=" + fileLockHolder.lockPath() + "]");
+        if (fileLockHolder != null) {
+            if (log.isDebugEnabled())
+                log.debug("Try to capture file lock [nodeId=" +
+                    cctx.localNodeId() + " path=" + fileLockHolder.lockPath() + "]");
 
-        fileLockHolder.tryLock(lockWaitTime);
+            fileLockHolder.tryLock(lockWaitTime);
+        }
     }
 
     /** {@inheritDoc} */
     @Override public void unLock() {
-        if (log.isDebugEnabled())
-            log.debug("Release file lock [nodeId=" +
-                cctx.localNodeId() + " path=" + fileLockHolder.lockPath() + "]");
+        if (fileLockHolder != null) {
+            if (log.isDebugEnabled())
+                log.debug("Release file lock [nodeId=" +
+                    cctx.localNodeId() + " path=" + fileLockHolder.lockPath() + "]");
 
-        fileLockHolder.release();
+            fileLockHolder.release();
+        }
     }
 
     /** {@inheritDoc} */
@@ -628,7 +646,8 @@ public class GridCacheDatabaseSharedManager extends IgniteCacheDatabaseSharedMan
         if (!cctx.kernalContext().clientNode()) {
             unLock();
 
-            fileLockHolder.close();
+            if (fileLockHolder != null)
+                fileLockHolder.close();
         }
 
         unRegistrateMetricsMBean();
@@ -875,7 +894,7 @@ public class GridCacheDatabaseSharedManager extends IgniteCacheDatabaseSharedMan
                             idxRebuildFuts.remove(cacheId, rebuildFut);
 
                             log().info("Finished indexes rebuilding for cache: [name=" + cacheCtx.config().getName()
-                                    + ", grpName=" + cacheCtx.config().getGroupName());
+                                + ", grpName=" + cacheCtx.config().getGroupName());
                         }
                     });
                 }
@@ -1189,7 +1208,6 @@ public class GridCacheDatabaseSharedManager extends IgniteCacheDatabaseSharedMan
 
     /**
      * For debugging only. TODO: remove.
-     *
      */
     public Map<T2<Integer, Integer>, T2<Long, WALPointer>> reservedForPreloading() {
         return reservedForPreloading;
@@ -2310,7 +2328,6 @@ public class GridCacheDatabaseSharedManager extends IgniteCacheDatabaseSharedMan
                     cpRec.addCacheGroupState(grp.groupId(), state);
                 }
 
-
                 cpPagesTuple = beginAllCheckpoints();
 
                 hasPages = hasPageForWrite(cpPagesTuple.get1());
@@ -2400,7 +2417,8 @@ public class GridCacheDatabaseSharedManager extends IgniteCacheDatabaseSharedMan
         }
 
         /**
-         * @return tuple with collections of FullPageIds obtained from each PageMemory and overall number of dirty pages.
+         * @return tuple with collections of FullPageIds obtained from each PageMemory and overall number of dirty
+         * pages.
          */
         private IgniteBiTuple<Collection<GridMultiCollectionWrapper<FullPageId>>, Integer> beginAllCheckpoints() {
             Collection<GridMultiCollectionWrapper<FullPageId>> res = new ArrayList(memoryPolicies().size());
@@ -2792,9 +2810,9 @@ public class GridCacheDatabaseSharedManager extends IgniteCacheDatabaseSharedMan
     }
 
     /**
-     * Checkpoint history. Holds chronological ordered map with {@link GridCacheDatabaseSharedManager.CheckpointEntry CheckpointEntries}.
-     * Data is loaded from corresponding checkpoint directory.
-     * This directory holds files for checkpoint start and end.
+     * Checkpoint history. Holds chronological ordered map with {@link GridCacheDatabaseSharedManager.CheckpointEntry
+     * CheckpointEntries}. Data is loaded from corresponding checkpoint directory. This directory holds files for
+     * checkpoint start and end.
      */
     @SuppressWarnings("PublicInnerClass")
     public class CheckpointHistory {
@@ -3117,7 +3135,7 @@ public class GridCacheDatabaseSharedManager extends IgniteCacheDatabaseSharedMan
     /**
      *
      */
-    private static class FileLockHolder {
+    public static class FileLockHolder implements AutoCloseable {
         /** Lock file name. */
         private static final String lockFileName = "lock";
 
@@ -3130,8 +3148,8 @@ public class GridCacheDatabaseSharedManager extends IgniteCacheDatabaseSharedMan
         /** Lock. */
         private FileLock lock;
 
-        /** Id. */
-        private GridKernalContext ctx;
+        /** Kernal context to generate Id of locked node in file. */
+        @NotNull private GridKernalContext ctx;
 
         /** Logger. */
         private IgniteLogger log;
@@ -3139,7 +3157,7 @@ public class GridCacheDatabaseSharedManager extends IgniteCacheDatabaseSharedMan
         /**
          * @param path Path.
          */
-        private FileLockHolder(String path, GridKernalContext ctx, IgniteLogger log) {
+        public FileLockHolder(String path, @NotNull GridKernalContext ctx, IgniteLogger log) {
             try {
                 file = Paths.get(path, lockFileName).toFile();
 
@@ -3168,7 +3186,14 @@ public class GridCacheDatabaseSharedManager extends IgniteCacheDatabaseSharedMan
             sb.a("[").a(ctx.localNodeId().toString()).a("]");
 
             //write ip addresses
-            sb.a(ctx.discovery().localNode().addresses());
+            final GridDiscoveryManager discovery = ctx.discovery();
+
+            if (discovery != null) { //discovery may be not up and running
+                final ClusterNode node = discovery.localNode();
+
+                if (node != null)
+                    sb.a(node.addresses());
+            }
 
             //write ports
             sb.a("[");
@@ -3264,17 +3289,13 @@ public class GridCacheDatabaseSharedManager extends IgniteCacheDatabaseSharedMan
             return content;
         }
 
-        /**
-         *
-         */
-        private void release() {
+        /** Releases file lock */
+        public void release() {
             U.releaseQuiet(lock);
         }
 
-        /**
-         *
-         */
-        private void close() {
+        /** Closes file channel */
+        public void close() {
             U.closeQuiet(lockFile);
         }
 

http://git-wip-us.apache.org/repos/asf/ignite/blob/62f3c4c5/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/IgniteCacheDatabaseSharedManager.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/IgniteCacheDatabaseSharedManager.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/IgniteCacheDatabaseSharedManager.java
index 6ea6eff..d7682f0 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/IgniteCacheDatabaseSharedManager.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/IgniteCacheDatabaseSharedManager.java
@@ -50,6 +50,7 @@ import org.apache.ignite.internal.processors.cache.persistence.evict.NoOpPageEvi
 import org.apache.ignite.internal.processors.cache.persistence.evict.PageEvictionTracker;
 import org.apache.ignite.internal.processors.cache.persistence.evict.Random2LruPageEvictionTracker;
 import org.apache.ignite.internal.processors.cache.persistence.evict.RandomLruPageEvictionTracker;
+import org.apache.ignite.internal.processors.cache.persistence.filename.PdsFolderSettings;
 import org.apache.ignite.internal.processors.cache.persistence.freelist.FreeList;
 import org.apache.ignite.internal.processors.cache.persistence.freelist.FreeListImpl;
 import org.apache.ignite.internal.processors.cache.persistence.tree.reuse.ReuseList;
@@ -939,17 +940,21 @@ public class IgniteCacheDatabaseSharedManager extends GridCacheSharedManagerAdap
      *
      * @throws IgniteCheckedException If resolving swap directory fails.
      */
-    @Nullable protected File buildAllocPath(MemoryPolicyConfiguration plc) throws IgniteCheckedException {
+    @Nullable private File buildAllocPath(MemoryPolicyConfiguration plc) throws IgniteCheckedException {
         String path = plc.getSwapFilePath();
 
         if (path == null)
             return null;
 
-        String consId = String.valueOf(cctx.discovery().consistentId());
+        final PdsFolderSettings folderSettings = cctx.kernalContext().pdsFolderResolver().resolveFolders();
+        final String folderName;
 
-        consId = consId.replaceAll("[:,\\.]", "_");
+        if(folderSettings.isCompatible())
+            folderName = String.valueOf(folderSettings.consistentId()).replaceAll("[:,\\.]", "_");
+        else
+            folderName = folderSettings.folderName();
 
-        return buildPath(path, consId);
+        return buildPath(path, folderName);
     }
 
     /**

http://git-wip-us.apache.org/repos/asf/ignite/blob/62f3c4c5/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/file/FilePageStoreManager.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/file/FilePageStoreManager.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/file/FilePageStoreManager.java
index a20bda1..ed82127 100755
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/file/FilePageStoreManager.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/file/FilePageStoreManager.java
@@ -47,6 +47,7 @@ import org.apache.ignite.internal.processors.cache.CacheGroupContext;
 import org.apache.ignite.internal.processors.cache.CacheGroupDescriptor;
 import org.apache.ignite.internal.processors.cache.GridCacheSharedManagerAdapter;
 import org.apache.ignite.internal.processors.cache.StoredCacheData;
+import org.apache.ignite.internal.processors.cache.persistence.filename.PdsFolderSettings;
 import org.apache.ignite.internal.processors.cache.persistence.snapshot.IgniteCacheSnapshotManager;
 import org.apache.ignite.internal.util.typedef.internal.U;
 import org.apache.ignite.marshaller.Marshaller;
@@ -93,7 +94,7 @@ public class FilePageStoreManager extends GridCacheSharedManagerAdapter implemen
     /** */
     private PersistentStoreConfiguration pstCfg;
 
-    /** Absolute directory for file page store */
+    /** Absolute directory for file page store. Includes consistent id based folder. */
     private File storeWorkDir;
 
     /** */
@@ -117,29 +118,13 @@ public class FilePageStoreManager extends GridCacheSharedManagerAdapter implemen
 
     /** {@inheritDoc} */
     @Override public void start0() throws IgniteCheckedException {
-        if (cctx.kernalContext().clientNode())
+        final GridKernalContext ctx = cctx.kernalContext();
+        if (ctx.clientNode())
             return;
 
-        String consId = U.maskForFileName(cctx.kernalContext().discovery().consistentId().toString());
-
-        if (pstCfg.getPersistentStorePath() != null) {
-            File workDir0 = new File(pstCfg.getPersistentStorePath());
-
-            if (!workDir0.isAbsolute())
-                workDir0 = U.resolveWorkDirectory(
-                    igniteCfg.getWorkDirectory(),
-                    pstCfg.getPersistentStorePath(),
-                    false
-                );
+        final PdsFolderSettings folderSettings = ctx.pdsFolderResolver().resolveFolders();
 
-            storeWorkDir = new File(workDir0, consId);
-        }
-        else
-            storeWorkDir = new File(U.resolveWorkDirectory(
-                igniteCfg.getWorkDirectory(),
-                DFLT_STORE_DIR,
-                false
-            ), consId);
+        storeWorkDir = new File(folderSettings.persistentStoreRootPath(), folderSettings.folderName());
 
         U.ensureDirectory(storeWorkDir, "page store work directory", log);
     }
@@ -551,7 +536,7 @@ public class FilePageStoreManager extends GridCacheSharedManagerAdapter implemen
     }
 
     /**
-     * @return Store work dir.
+     * @return Store work dir. Includes consistent-id based folder
      */
     public File workDir() {
         return storeWorkDir;

http://git-wip-us.apache.org/repos/asf/ignite/blob/62f3c4c5/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/filename/PdsConsistentIdProcessor.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/filename/PdsConsistentIdProcessor.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/filename/PdsConsistentIdProcessor.java
new file mode 100644
index 0000000..c73a952
--- /dev/null
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/filename/PdsConsistentIdProcessor.java
@@ -0,0 +1,568 @@
+/*
+ * 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.persistence.filename;
+
+import java.io.File;
+import java.io.FileFilter;
+import java.io.IOException;
+import java.io.Serializable;
+import java.text.SimpleDateFormat;
+import java.util.ArrayList;
+import java.util.Collections;
+import java.util.Comparator;
+import java.util.List;
+import java.util.UUID;
+import java.util.regex.Matcher;
+import java.util.regex.Pattern;
+import org.apache.ignite.IgniteCheckedException;
+import org.apache.ignite.IgniteLogger;
+import org.apache.ignite.configuration.IgniteConfiguration;
+import org.apache.ignite.configuration.PersistentStoreConfiguration;
+import org.apache.ignite.internal.GridKernalContext;
+import org.apache.ignite.internal.processors.GridProcessorAdapter;
+import org.apache.ignite.internal.processors.cache.persistence.GridCacheDatabaseSharedManager;
+import org.apache.ignite.internal.util.typedef.internal.A;
+import org.apache.ignite.internal.util.typedef.internal.SB;
+import org.apache.ignite.internal.util.typedef.internal.U;
+import org.jetbrains.annotations.NotNull;
+import org.jetbrains.annotations.Nullable;
+
+import static org.apache.ignite.IgniteSystemProperties.IGNITE_DATA_STORAGE_FOLDER_BY_CONSISTENT_ID;
+import static org.apache.ignite.IgniteSystemProperties.getBoolean;
+
+/**
+ * Component for resolving PDS storage file names, also used for generating consistent ID for case PDS mode is enabled
+ */
+public class PdsConsistentIdProcessor extends GridProcessorAdapter implements PdsFoldersResolver {
+    /** Database subfolders constant prefix. */
+    private static final String DB_FOLDER_PREFIX = "node";
+
+    /** Node index and uid separator in subfolders name. */
+    private static final String NODEIDX_UID_SEPARATOR = "-";
+
+    /** Constant node subfolder prefix and node index pattern (nodeII, where II - node index as decimal integer) */
+    private static final String NODE_PATTERN = DB_FOLDER_PREFIX + "[0-9]*" + NODEIDX_UID_SEPARATOR;
+
+    /** Uuid as string pattern. */
+    private static final String UUID_STR_PATTERN = "[a-fA-F0-9]{8}-[a-fA-F0-9]{4}-[a-fA-F0-9]{4}-[a-fA-F0-9]{4}-[a-fA-F0-9]{12}";
+
+    /**
+     * Subdir (nodeII-UID, where II - node index as decimal integer, UID - string representation of consistent ID)
+     * pattern.
+     */
+    private static final String SUBDIR_PATTERN = NODE_PATTERN + UUID_STR_PATTERN;
+
+    /** Database subfolders for new style filter. */
+    public static final FileFilter DB_SUBFOLDERS_NEW_STYLE_FILTER = new FileFilter() {
+        @Override public boolean accept(File pathname) {
+            return pathname.isDirectory() && pathname.getName().matches(SUBDIR_PATTERN);
+        }
+    };
+
+    /** Database subfolders for old style filter. */
+    private static final FileFilter DB_SUBFOLDERS_OLD_STYLE_FILTER = new FileFilter() {
+        @Override public boolean accept(File pathname) {
+            return pathname.isDirectory()
+                && !"wal".equals(pathname.getName())
+                && !pathname.getName().matches(SUBDIR_PATTERN);
+        }
+    };
+
+    /** Database default folder. */
+    public static final String DB_DEFAULT_FOLDER = "db";
+
+    /** Config. */
+    private IgniteConfiguration cfg;
+
+    /** Logger. */
+    private IgniteLogger log;
+
+    /** Context. */
+    private GridKernalContext ctx;
+
+    /** Cached folder settings. */
+    private PdsFolderSettings settings;
+
+    /**
+     * Creates folders resolver
+     *
+     * @param ctx Context.
+     */
+    public PdsConsistentIdProcessor(final GridKernalContext ctx) {
+        super(ctx);
+
+        this.cfg = ctx.config();
+        this.log = ctx.log(PdsFoldersResolver.class);
+        this.ctx = ctx;
+    }
+
+    /**
+     * Prepares compatible PDS folder settings. No locking is performed, consistent ID is not overridden.
+     *
+     * @param pstStoreBasePath DB storage base path or null if persistence is not enabled.
+     * @param consistentId compatibility consistent ID
+     * @return PDS folder settings compatible with previous versions.
+     */
+    private PdsFolderSettings compatibleResolve(
+        @Nullable final File pstStoreBasePath,
+        @NotNull final Serializable consistentId) {
+
+        if (cfg.getConsistentId() != null) {
+            // compatible mode from configuration is used fot this case, no locking, no consitent id change
+            return new PdsFolderSettings(pstStoreBasePath, cfg.getConsistentId());
+        }
+
+        return new PdsFolderSettings(pstStoreBasePath, consistentId);
+    }
+
+    /** {@inheritDoc} */
+    @Override public PdsFolderSettings resolveFolders() throws IgniteCheckedException {
+        if (settings == null) {
+            settings = prepareNewSettings();
+
+            if (!settings.isCompatible()) {
+                if (log.isInfoEnabled())
+                    log.info("Consistent ID used for local node is [" + settings.consistentId() + "] " +
+                        "according to persistence data storage folders");
+
+                ctx.discovery().consistentId(settings.consistentId());
+            }
+        }
+        return settings;
+    }
+
+    /**
+     * Creates new settings when we don't have cached one.
+     *
+     * @return new settings with prelocked directory (if appropriate).
+     * @throws IgniteCheckedException if IO failed.
+     */
+    private PdsFolderSettings prepareNewSettings() throws IgniteCheckedException {
+        final File pstStoreBasePath = resolvePersistentStoreBasePath();
+        //here deprecated method is used to get compatible version of consistentId
+        final Serializable consistentId = ctx.discovery().consistentId();
+
+        if (!cfg.isPersistentStoreEnabled())
+            return compatibleResolve(pstStoreBasePath, consistentId);
+
+        if (getBoolean(IGNITE_DATA_STORAGE_FOLDER_BY_CONSISTENT_ID, false))
+            return compatibleResolve(pstStoreBasePath, consistentId);
+
+        // compatible mode from configuration is used fot this case
+        if (cfg.getConsistentId() != null) {
+            // compatible mode from configuration is used fot this case, no locking, no consistent id change
+            return new PdsFolderSettings(pstStoreBasePath, cfg.getConsistentId());
+        }
+        // The node scans the work directory and checks if there is a folder matching the consistent ID.
+        // If such a folder exists, we start up with this ID (compatibility mode)
+        final String subFolder = U.maskForFileName(consistentId.toString());
+
+        final GridCacheDatabaseSharedManager.FileLockHolder oldStyleFolderLockHolder = tryLock(new File(pstStoreBasePath, subFolder));
+
+        if (oldStyleFolderLockHolder != null)
+            return new PdsFolderSettings(pstStoreBasePath,
+                subFolder,
+                consistentId,
+                oldStyleFolderLockHolder,
+                true);
+
+        final File[] oldStyleFolders = pstStoreBasePath.listFiles(DB_SUBFOLDERS_OLD_STYLE_FILTER);
+
+        if (oldStyleFolders != null && oldStyleFolders.length != 0) {
+            for (File folder : oldStyleFolders) {
+                final String path = getPathDisplayableInfo(folder);
+
+                U.warn(log, "There is other non-empty storage folder under storage base directory [" + path + "]");
+            }
+        }
+
+        for (FolderCandidate next : getNodeIndexSortedCandidates(pstStoreBasePath)) {
+            final GridCacheDatabaseSharedManager.FileLockHolder fileLockHolder = tryLock(next.subFolderFile());
+
+            if (fileLockHolder != null) {
+                if (log.isInfoEnabled())
+                    log.info("Successfully locked persistence storage folder [" + next.subFolderFile() + "]");
+
+                return new PdsFolderSettings(pstStoreBasePath,
+                    next.subFolderFile().getName(),
+                    next.uuid(),
+                    fileLockHolder,
+                    false);
+            }
+        }
+
+        // was not able to find free slot, allocating new
+        final GridCacheDatabaseSharedManager.FileLockHolder rootDirLock = lockRootDirectory(pstStoreBasePath);
+
+        try {
+            final List<FolderCandidate> sortedCandidates = getNodeIndexSortedCandidates(pstStoreBasePath);
+            final int nodeIdx = sortedCandidates.isEmpty() ? 0 : (sortedCandidates.get(sortedCandidates.size() - 1).nodeIndex() + 1);
+
+            return generateAndLockNewDbStorage(pstStoreBasePath, nodeIdx);
+        }
+        finally {
+            rootDirLock.release();
+            rootDirLock.close();
+        }
+    }
+
+    /**
+     * Calculate overall folder size.
+     *
+     * @param dir directory to scan.
+     * @return total size in bytes.
+     */
+    private static FolderParams folderSize(File dir) {
+        final FolderParams params = new FolderParams();
+
+        visitFolder(dir, params);
+
+        return params;
+    }
+
+    /**
+     * Scans provided directory and its sub dirs, collects found metrics.
+     *
+     * @param dir directory to start scan from.
+     * @param params input/output.
+     */
+    private static void visitFolder(final File dir, final FolderParams params) {
+        for (File file : dir.listFiles()) {
+            if (file.isDirectory())
+                visitFolder(file, params);
+            else {
+                params.size += file.length();
+                params.lastModified = Math.max(params.lastModified, dir.lastModified());
+            }
+        }
+    }
+
+    /**
+     * @param folder folder to scan.
+     * @return folder displayable information.
+     */
+    @NotNull private String getPathDisplayableInfo(final File folder) {
+        final SB res = new SB();
+
+        res.a(getCanonicalPath(folder));
+        res.a(", ");
+        final FolderParams params = folderSize(folder);
+
+        res.a(params.size);
+        res.a(" bytes, modified ");
+        final SimpleDateFormat simpleDateFormat = new SimpleDateFormat("MM/dd/yyyy hh:mm a");
+
+        res.a(simpleDateFormat.format(params.lastModified));
+        res.a(" ");
+
+        return res.toString();
+    }
+
+    /**
+     * Returns the canonical pathname string of this abstract pathname.
+     *
+     * @param file path to convert.
+     * @return canonical pathname or at leas absolute if convert to canonical failed.
+     */
+    @NotNull private String getCanonicalPath(final File file) {
+        try {
+            return file.getCanonicalPath();
+        }
+        catch (IOException ignored) {
+            return file.getAbsolutePath();
+        }
+    }
+
+    /**
+     * Pad start of string with provided character.
+     *
+     * @param str sting to pad.
+     * @param minLength expected length.
+     * @param padChar padding character.
+     * @return padded string.
+     */
+    private static String padStart(String str, int minLength, char padChar) {
+        A.notNull(str, "String should not be empty");
+        if (str.length() >= minLength)
+            return str;
+
+        final SB sb = new SB(minLength);
+
+        for (int i = str.length(); i < minLength; ++i)
+            sb.a(padChar);
+
+        sb.a(str);
+
+        return sb.toString();
+
+    }
+
+    /**
+     * Creates new DB storage folder.
+     *
+     * @param pstStoreBasePath DB root path.
+     * @param nodeIdx next node index to use in folder name.
+     * @return new settings to be used in this node.
+     * @throws IgniteCheckedException if failed.
+     */
+    @NotNull private PdsFolderSettings generateAndLockNewDbStorage(final File pstStoreBasePath,
+        final int nodeIdx) throws IgniteCheckedException {
+
+        final UUID uuid = UUID.randomUUID();
+        final String consIdBasedFolder = genNewStyleSubfolderName(nodeIdx, uuid);
+        final File newRandomFolder = U.resolveWorkDirectory(pstStoreBasePath.getAbsolutePath(), consIdBasedFolder, false); //mkdir here
+        final GridCacheDatabaseSharedManager.FileLockHolder fileLockHolder = tryLock(newRandomFolder);
+
+        if (fileLockHolder != null) {
+            if (log.isInfoEnabled())
+                log.info("Successfully created new persistent storage folder [" + newRandomFolder + "]");
+
+            return new PdsFolderSettings(pstStoreBasePath, consIdBasedFolder, uuid, fileLockHolder, false);
+        }
+        throw new IgniteCheckedException("Unable to lock file generated randomly [" + newRandomFolder + "]");
+    }
+
+    /**
+     * Generates DB subfolder name for provided node index (local) and UUID (consistent ID)
+     *
+     * @param nodeIdx node index.
+     * @param uuid consistent ID.
+     * @return folder file name
+     */
+    @NotNull public static String genNewStyleSubfolderName(final int nodeIdx, final UUID uuid) {
+        final String uuidAsStr = uuid.toString();
+
+        assert uuidAsStr.matches(UUID_STR_PATTERN);
+
+        final String nodeIdxPadded = padStart(Integer.toString(nodeIdx), 2, '0');
+
+        return DB_FOLDER_PREFIX + nodeIdxPadded + NODEIDX_UID_SEPARATOR + uuidAsStr;
+    }
+
+    /**
+     * Acquires lock to root storage directory, used to lock root directory in case creating new files is required.
+     *
+     * @param pstStoreBasePath rood DB dir to lock
+     * @return locked directory, should be released and closed later
+     * @throws IgniteCheckedException if failed
+     */
+    @NotNull private GridCacheDatabaseSharedManager.FileLockHolder lockRootDirectory(File pstStoreBasePath)
+        throws IgniteCheckedException {
+
+        GridCacheDatabaseSharedManager.FileLockHolder rootDirLock;
+        int retry = 0;
+
+        while ((rootDirLock = tryLock(pstStoreBasePath)) == null) {
+            if (retry > 600)
+                throw new IgniteCheckedException("Unable to start under DB storage path [" + pstStoreBasePath + "]" +
+                    ". Lock is being held to root directory");
+            retry++;
+        }
+
+        return rootDirLock;
+    }
+
+    /**
+     * @param pstStoreBasePath root storage folder to scan.
+     * @return empty list if there is no files in folder to test. Non null value is returned for folder having
+     * applicable new style files. Collection is sorted ascending according to node ID, 0 node index is coming first.
+     */
+    @Nullable private List<FolderCandidate> getNodeIndexSortedCandidates(File pstStoreBasePath) {
+        final File[] files = pstStoreBasePath.listFiles(DB_SUBFOLDERS_NEW_STYLE_FILTER);
+
+        if (files == null)
+            return Collections.emptyList();
+
+        final List<FolderCandidate> res = new ArrayList<>();
+
+        for (File file : files) {
+            final FolderCandidate candidate = parseFileName(file);
+
+            if (candidate != null)
+                res.add(candidate);
+        }
+        Collections.sort(res, new Comparator<FolderCandidate>() {
+            @Override public int compare(FolderCandidate c1, FolderCandidate c2) {
+                return Integer.compare(c1.nodeIndex(), c2.nodeIndex());
+            }
+        });
+
+        return res;
+    }
+
+    /**
+     * Tries to lock subfolder within storage root folder.
+     *
+     * @param dbStoreDirWithSubdirectory DB store directory, is to be absolute and should include consistent ID based
+     * sub folder.
+     * @return non null holder if lock was successful, null in case lock failed. If directory does not exist method will
+     * always fail to lock.
+     */
+    private GridCacheDatabaseSharedManager.FileLockHolder tryLock(File dbStoreDirWithSubdirectory) {
+        if (!dbStoreDirWithSubdirectory.exists())
+            return null;
+
+        final String path = dbStoreDirWithSubdirectory.getAbsolutePath();
+        final GridCacheDatabaseSharedManager.FileLockHolder fileLockHolder
+            = new GridCacheDatabaseSharedManager.FileLockHolder(path, ctx, log);
+
+        try {
+            fileLockHolder.tryLock(1000);
+
+            return fileLockHolder;
+        }
+        catch (IgniteCheckedException e) {
+            U.closeQuiet(fileLockHolder);
+
+            if (log.isInfoEnabled())
+                log.info("Unable to acquire lock to file [" + path + "], reason: " + e.getMessage());
+
+            return null;
+        }
+    }
+
+    /**
+     * @return DB storage absolute root path resolved as 'db' folder in Ignite work dir (by default) or using persistent
+     * store configuration. Null if persistence is not enabled. Returned folder is created automatically.
+     * @throws IgniteCheckedException if I/O failed.
+     */
+    @Nullable private File resolvePersistentStoreBasePath() throws IgniteCheckedException {
+        final PersistentStoreConfiguration pstCfg = cfg.getPersistentStoreConfiguration();
+
+        if (pstCfg == null)
+            return null;
+
+        final String pstPath = pstCfg.getPersistentStorePath();
+
+        return U.resolveWorkDirectory(
+            cfg.getWorkDirectory(),
+            pstPath != null ? pstPath : DB_DEFAULT_FOLDER,
+            false
+        );
+
+    }
+
+    /**
+     * @param subFolderFile new style folder name to parse
+     * @return Pair of UUID and node index
+     */
+    private FolderCandidate parseFileName(@NotNull final File subFolderFile) {
+        return parseSubFolderName(subFolderFile, log);
+    }
+
+    /**
+     * @param subFolderFile new style file to parse.
+     * @param log Logger.
+     * @return Pair of UUID and node index.
+     */
+    @Nullable public static FolderCandidate parseSubFolderName(
+        @NotNull final File subFolderFile,
+        @NotNull final IgniteLogger log) {
+
+        final String fileName = subFolderFile.getName();
+        final Matcher matcher = Pattern.compile(NODE_PATTERN).matcher(fileName);
+        if (!matcher.find())
+            return null;
+
+        int uidStart = matcher.end();
+
+        try {
+            final String uid = fileName.substring(uidStart);
+            final UUID uuid = UUID.fromString(uid);
+            final String substring = fileName.substring(DB_FOLDER_PREFIX.length(), uidStart - NODEIDX_UID_SEPARATOR.length());
+            final int idx = Integer.parseInt(substring);
+
+            return new FolderCandidate(subFolderFile, idx, uuid);
+        }
+        catch (Exception e) {
+            U.warn(log, "Unable to parse new style file format from [" + subFolderFile.getAbsolutePath() + "]: " + e);
+
+            return null;
+        }
+    }
+
+    /** {@inheritDoc} */
+    @Override public void stop(boolean cancel) throws IgniteCheckedException {
+        if (settings != null) {
+            final GridCacheDatabaseSharedManager.FileLockHolder fileLockHolder = settings.getLockedFileLockHolder();
+
+            if (fileLockHolder != null) {
+                fileLockHolder.release();
+                fileLockHolder.close();
+            }
+        }
+        super.stop(cancel);
+    }
+
+    /** Path metrics */
+    private static class FolderParams {
+        /** Overall size in bytes. */
+        private long size;
+
+        /** Last modified. */
+        private long lastModified;
+    }
+
+    /**
+     * Represents parsed new style file and encoded parameters in this file name
+     */
+    public static class FolderCandidate {
+        /** Absolute file path pointing to DB subfolder within DB storage root folder. */
+        private final File subFolderFile;
+
+        /** Node index (local, usually 0 if multiple nodes are not started at local PC). */
+        private final int nodeIdx;
+
+        /** Uuid contained in file name, is to be set as consistent ID. */
+        private final UUID uuid;
+
+        /**
+         * @param subFolderFile Absolute file path pointing to DB subfolder.
+         * @param nodeIdx Node index.
+         * @param uuid Uuid.
+         */
+        public FolderCandidate(File subFolderFile, int nodeIdx, UUID uuid) {
+            this.subFolderFile = subFolderFile;
+            this.nodeIdx = nodeIdx;
+            this.uuid = uuid;
+        }
+
+        /**
+         * @return Node index (local, usually 0 if multiple nodes are not started at local PC).
+         */
+        public int nodeIndex() {
+            return nodeIdx;
+        }
+
+        /**
+         * @return Uuid contained in file name, is to be set as consistent ID.
+         */
+        public Serializable uuid() {
+            return uuid;
+        }
+
+        /**
+         * @return Absolute file path pointing to DB subfolder within DB storage root folder.
+         */
+        public File subFolderFile() {
+            return subFolderFile;
+        }
+    }
+}
+
+

http://git-wip-us.apache.org/repos/asf/ignite/blob/62f3c4c5/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/filename/PdsFolderSettings.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/filename/PdsFolderSettings.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/filename/PdsFolderSettings.java
new file mode 100644
index 0000000..20fb5ca
--- /dev/null
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/filename/PdsFolderSettings.java
@@ -0,0 +1,138 @@
+/*
+ * 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.persistence.filename;
+
+import java.io.File;
+import java.io.Serializable;
+import org.apache.ignite.internal.processors.cache.persistence.GridCacheDatabaseSharedManager;
+import org.apache.ignite.internal.util.typedef.internal.U;
+import org.jetbrains.annotations.NotNull;
+import org.jetbrains.annotations.Nullable;
+
+/**
+ * Class holds information required for folder generation for ignite persistent store
+ */
+public class PdsFolderSettings {
+    /**
+     * DB storage absolute root path resolved as 'db' folder in Ignite work dir (by default) or using persistent store
+     * configuration. <br>
+     * Note WAL storage may be configured outside this path.<br>
+     * This value may be null if persistence is not enabled.
+     */
+    @Nullable private final File persistentStoreRootPath;
+
+    /** Sub folder name containing consistent ID and optionally node index. */
+    private final String folderName;
+
+    /** Consistent id to be set to local node. */
+    private final Serializable consistentId;
+
+    /**
+     * File lock holder with prelocked db directory. For non compatible mode this holder contains prelocked work
+     * directory. This value is to be used at activate instead of locking. <br> May be null in case preconfigured
+     * consistent ID is used or in case lock holder was already taken by other processor.
+     */
+    @Nullable private final GridCacheDatabaseSharedManager.FileLockHolder fileLockHolder;
+
+    /**
+     * Indicates if compatible mode is enabled, in that case all sub folders are generated from consistent ID without
+     * 'node' and node index prefix. In compatible mode there is no overriding for consistent ID is done.
+     */
+    private final boolean compatible;
+
+    /**
+     * Creates settings in for new PST(DB) folder mode.
+     *
+     * @param persistentStoreRootPath Persistent store root path or null if non PDS mode.
+     * @param folderName Sub folder name containing consistent ID and optionally node index.
+     * @param consistentId Consistent id.
+     * @param fileLockHolder File lock holder with prelocked db directory.
+     * @param compatible Compatible mode flag.
+     */
+    public PdsFolderSettings(@Nullable final File persistentStoreRootPath,
+        final String folderName,
+        final Serializable consistentId,
+        @Nullable final GridCacheDatabaseSharedManager.FileLockHolder fileLockHolder,
+        final boolean compatible) {
+
+        this.consistentId = consistentId;
+        this.folderName = folderName;
+        this.fileLockHolder = fileLockHolder;
+        this.compatible = compatible;
+        this.persistentStoreRootPath = persistentStoreRootPath;
+    }
+
+    /**
+     * Creates settings for compatible mode. Folder name is consistent ID (masked), no node prefix is added.
+     *
+     * @param persistentStoreRootPath root DB path.
+     * @param consistentId node consistent ID.
+     */
+    public PdsFolderSettings(
+        @Nullable final File persistentStoreRootPath,
+        @NotNull final Serializable consistentId) {
+
+        this.consistentId = consistentId;
+        this.compatible = true;
+        this.folderName = U.maskForFileName(consistentId.toString());
+        this.persistentStoreRootPath = persistentStoreRootPath;
+        this.fileLockHolder = null;
+    }
+
+    /**
+     * @return sub folders name based on consistent ID. In compatible mode this is escaped consistent ID, in new mode
+     * this is UUID based folder name.
+     */
+    public String folderName() {
+        return folderName;
+    }
+
+    /**
+     * @return Consistent id to be set to local node.
+     */
+    public Serializable consistentId() {
+        return consistentId;
+    }
+
+    /**
+     * @return flag indicating if compatible mode is enabled for folder generation. In that case all sub folders names are
+     * generated from consistent ID without 'node' and node index prefix. In compatible mode there is no overriding for
+     * consistent ID is done for cluster node. Locking files is independent to compatibility mode.
+     */
+    public boolean isCompatible() {
+        return compatible;
+    }
+
+    /**
+     * Returns already locked file lock holder to lock file in {@link #persistentStoreRootPath}. Unlock and close this
+     * lock is not required.
+     *
+     * @return File lock holder with prelocked db directory.
+     */
+    @Nullable public GridCacheDatabaseSharedManager.FileLockHolder getLockedFileLockHolder() {
+        return fileLockHolder;
+    }
+
+    /**
+     * @return DB storage absolute root path resolved as 'db' folder in Ignite work dir (by default) or using persistent
+     * store configuration. Note WAL storage may be configured outside this path. May return null for non pds mode.
+     */
+    @Nullable public File persistentStoreRootPath() {
+        return persistentStoreRootPath;
+    }
+}

http://git-wip-us.apache.org/repos/asf/ignite/blob/62f3c4c5/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/filename/PdsFoldersResolver.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/filename/PdsFoldersResolver.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/filename/PdsFoldersResolver.java
new file mode 100644
index 0000000..cefaa04
--- /dev/null
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/filename/PdsFoldersResolver.java
@@ -0,0 +1,33 @@
+/*
+ * 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.persistence.filename;
+
+import org.apache.ignite.IgniteCheckedException;
+
+/**
+ * Resolves folders for PDS mode, may have side effect as setting random UUID as local node consistent ID.
+ */
+public interface PdsFoldersResolver {
+    /**
+     * Prepares and caches PDS folder settings. Subsequent call to this method will provide same settings.
+     *
+     * @return PDS folder settings, consistentID and prelocked DB file lock.
+     * @throws IgniteCheckedException if failed.
+     */
+    public PdsFolderSettings resolveFolders() throws IgniteCheckedException;
+}

http://git-wip-us.apache.org/repos/asf/ignite/blob/62f3c4c5/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/wal/FileWriteAheadLogManager.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/wal/FileWriteAheadLogManager.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/wal/FileWriteAheadLogManager.java
index c8715aa..6a75dd2 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/wal/FileWriteAheadLogManager.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/wal/FileWriteAheadLogManager.java
@@ -64,6 +64,7 @@ import org.apache.ignite.internal.processors.cache.persistence.GridCacheDatabase
 import org.apache.ignite.internal.processors.cache.persistence.PersistenceMetricsImpl;
 import org.apache.ignite.internal.processors.cache.persistence.file.FileIO;
 import org.apache.ignite.internal.processors.cache.persistence.file.FileIOFactory;
+import org.apache.ignite.internal.processors.cache.persistence.filename.PdsFolderSettings;
 import org.apache.ignite.internal.processors.cache.persistence.wal.crc.PureJavaCrc32;
 import org.apache.ignite.internal.processors.cache.persistence.wal.record.HeaderRecord;
 import org.apache.ignite.internal.processors.cache.persistence.wal.serializer.RecordDataV1Serializer;
@@ -162,7 +163,7 @@ public class FileWriteAheadLogManager extends GridCacheSharedManagerAdapter impl
     /** WAL archive directory (including consistent ID as subfolder) */
     private File walArchiveDir;
 
-    /** Serializer of latest version. */
+    /** Serializer of latest version, used to read header record and for write records */
     private RecordSerializer serializer;
 
     /** Serializer latest version to use. */
@@ -258,25 +259,21 @@ public class FileWriteAheadLogManager extends GridCacheSharedManagerAdapter impl
     /** {@inheritDoc} */
     @Override public void start0() throws IgniteCheckedException {
         if (!cctx.kernalContext().clientNode()) {
-            String consId = consistentId();
-
-            A.notNullOrEmpty(consId, "consistentId");
-
-            consId = U.maskForFileName(consId);
+            final PdsFolderSettings resolveFolders = cctx.kernalContext().pdsFolderResolver().resolveFolders();
 
             checkWalConfiguration();
 
             walWorkDir = initDirectory(
                 psCfg.getWalStorePath(),
                 PersistentStoreConfiguration.DFLT_WAL_STORE_PATH,
-                consId,
+                resolveFolders.folderName(),
                 "write ahead log work directory"
             );
 
             walArchiveDir = initDirectory(
                 psCfg.getWalArchivePath(),
                 PersistentStoreConfiguration.DFLT_WAL_ARCHIVE_PATH,
-                consId,
+                resolveFolders.folderName(),
                 "write ahead log archive directory"
             );
 
@@ -317,13 +314,6 @@ public class FileWriteAheadLogManager extends GridCacheSharedManagerAdapter impl
         }
     }
 
-    /**
-     * @return Consistent ID.
-     */
-    protected String consistentId() {
-        return cctx.discovery().consistentId().toString();
-    }
-
     /** {@inheritDoc} */
     @Override protected void stop0(boolean cancel) {
         final GridTimeoutProcessor.CancelableTask schedule = backgroundFlushSchedule;

http://git-wip-us.apache.org/repos/asf/ignite/blob/62f3c4c5/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/wal/reader/StandaloneGridKernalContext.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/wal/reader/StandaloneGridKernalContext.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/wal/reader/StandaloneGridKernalContext.java
index 07be8b4..e234766 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/wal/reader/StandaloneGridKernalContext.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/persistence/wal/reader/StandaloneGridKernalContext.java
@@ -18,6 +18,7 @@
 package org.apache.ignite.internal.processors.cache.persistence.wal.reader;
 
 import java.io.File;
+import java.io.Serializable;
 import java.lang.reflect.Field;
 import java.util.Iterator;
 import java.util.List;
@@ -47,6 +48,8 @@ import org.apache.ignite.internal.managers.loadbalancer.GridLoadBalancerManager;
 import org.apache.ignite.internal.processors.affinity.GridAffinityProcessor;
 import org.apache.ignite.internal.processors.cache.GridCacheProcessor;
 import org.apache.ignite.internal.processors.cache.binary.CacheObjectBinaryProcessorImpl;
+import org.apache.ignite.internal.processors.cache.persistence.filename.PdsFoldersResolver;
+import org.apache.ignite.internal.processors.cache.persistence.filename.PdsFolderSettings;
 import org.apache.ignite.internal.processors.cacheobject.IgniteCacheObjectProcessor;
 import org.apache.ignite.internal.processors.closure.GridClosureProcessor;
 import org.apache.ignite.internal.processors.cluster.ClusterProcessor;
@@ -79,6 +82,7 @@ import org.apache.ignite.internal.processors.timeout.GridTimeoutProcessor;
 import org.apache.ignite.internal.suggestions.GridPerformanceSuggestions;
 import org.apache.ignite.internal.util.IgniteExceptionRegistry;
 import org.apache.ignite.internal.util.StripedExecutor;
+import org.apache.ignite.internal.util.typedef.internal.U;
 import org.apache.ignite.marshaller.Marshaller;
 import org.apache.ignite.plugin.PluginNotFoundException;
 import org.apache.ignite.plugin.PluginProvider;
@@ -106,7 +110,7 @@ public class StandaloneGridKernalContext implements GridKernalContext {
     @Nullable private IgniteCacheObjectProcessor cacheObjProcessor;
 
     /** Marshaller context implementation. */
-    private MarshallerContextImpl marshallerContext;
+    private MarshallerContextImpl marshallerCtx;
 
     /**
      * @param log Logger.
@@ -130,13 +134,13 @@ public class StandaloneGridKernalContext implements GridKernalContext {
             throw new IllegalStateException("Must not fail on empty providers list.", e);
         }
 
-        this.marshallerContext = new MarshallerContextImpl(null);
+        this.marshallerCtx = new MarshallerContextImpl(null);
         this.cfg = prepareIgniteConfiguration();
         this.cacheObjProcessor = binaryMetadataFileStoreDir != null ? binaryProcessor(this, binaryMetadataFileStoreDir) : null;
 
         if (marshallerMappingFileStoreDir != null) {
-            marshallerContext.setMarshallerMappingFileStoreDir(marshallerMappingFileStoreDir);
-            marshallerContext.onMarshallerProcessorStarted(this, null);
+            marshallerCtx.setMarshallerMappingFileStoreDir(marshallerMappingFileStoreDir);
+            marshallerCtx.onMarshallerProcessorStarted(this, null);
         }
     }
 
@@ -176,7 +180,7 @@ public class StandaloneGridKernalContext implements GridKernalContext {
         PersistentStoreConfiguration pstCfg = new PersistentStoreConfiguration();
         cfg.setPersistentStoreConfiguration(pstCfg);
 
-        marshaller.setContext(marshallerContext);
+        marshaller.setContext(marshallerCtx);
 
         return cfg;
     }
@@ -392,11 +396,7 @@ public class StandaloneGridKernalContext implements GridKernalContext {
 
     /** {@inheritDoc} */
     @Override public GridDiscoveryManager discovery() {
-        return new GridDiscoveryManager(StandaloneGridKernalContext.this) {
-            @Override public Object consistentId() {
-                return ""; // some non null value is required
-            }
-        };
+        return new GridDiscoveryManager(this);
     }
 
     /** {@inheritDoc} */
@@ -579,7 +579,7 @@ public class StandaloneGridKernalContext implements GridKernalContext {
 
     /** {@inheritDoc} */
     @Override public MarshallerContextImpl marshallerContext() {
-        return marshallerContext;
+        return marshallerCtx;
     }
 
     /** {@inheritDoc} */
@@ -598,6 +598,16 @@ public class StandaloneGridKernalContext implements GridKernalContext {
     }
 
     /** {@inheritDoc} */
+    @Override public PdsFoldersResolver pdsFolderResolver() {
+        return new PdsFoldersResolver() {
+            /** {@inheritDoc} */
+            @Override public PdsFolderSettings resolveFolders() {
+                return new PdsFolderSettings(new File("."), U.maskForFileName(""));
+            }
+        };
+    }
+
+    /** {@inheritDoc} */
     @NotNull @Override public Iterator<GridComponent> iterator() {
         return null;
     }

http://git-wip-us.apache.org/repos/asf/ignite/blob/62f3c4c5/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 e6eaa8e..eb8ee30 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
@@ -966,10 +966,10 @@ public class TcpDiscoverySpi extends IgniteSpiAdapter implements DiscoverySpi {
 
             initAddresses();
 
-            Serializable cfgId = ignite.configuration().getConsistentId();
+            final Serializable cfgId = ignite.configuration().getConsistentId();
 
             if (cfgId == null) {
-                List<String> sortedAddrs = new ArrayList<>(addrs.get1());
+                final List<String> sortedAddrs = new ArrayList<>(addrs.get1());
 
                 Collections.sort(sortedAddrs);
 

http://git-wip-us.apache.org/repos/asf/ignite/blob/62f3c4c5/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/internal/TcpDiscoveryNode.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/internal/TcpDiscoveryNode.java b/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/internal/TcpDiscoveryNode.java
index 20fb6c5..38c2a1b 100644
--- a/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/internal/TcpDiscoveryNode.java
+++ b/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/internal/TcpDiscoveryNode.java
@@ -225,6 +225,21 @@ public class TcpDiscoveryNode extends GridMetadataAwareAdapter implements Cluste
         return consistentId;
     }
 
+    /**
+     * Sets consistent globally unique node ID which survives node restarts.
+     *
+     * @param consistentId Consistent globally unique node ID.
+     */
+    public void setConsistentId(Serializable consistentId) {
+        this.consistentId = consistentId;
+
+        final Map<String, Object> map = new HashMap<>(attrs);
+
+        map.put(ATTR_NODE_CONSISTENT_ID, consistentId);
+
+        attrs = Collections.unmodifiableMap(map);
+    }
+
     /** {@inheritDoc} */
     @SuppressWarnings("unchecked")
     @Override public <T> T attribute(String name) {