Explorar el Código

HDFS-13821. RBF: Add dfs.federation.router.mount-table.cache.enable so that users can disable cache. Contributed by Fei Hui.

(cherry picked from commit 81847392badcd58d934333e7c3b5bf14b4fa1f3f)
Yiqun Lin hace 6 años
padre
commit
8e09ff918e

+ 27 - 10
hadoop-hdfs-project/hadoop-hdfs-rbf/src/main/java/org/apache/hadoop/hdfs/server/federation/resolver/MountTableResolver.java

@@ -22,6 +22,8 @@ import static org.apache.hadoop.hdfs.client.HdfsClientConfigKeys.DeprecatedKeys.
 import static org.apache.hadoop.hdfs.server.federation.router.RBFConfigKeys.DFS_ROUTER_DEFAULT_NAMESERVICE;
 import static org.apache.hadoop.hdfs.server.federation.router.RBFConfigKeys.DFS_ROUTER_DEFAULT_NAMESERVICE;
 import static org.apache.hadoop.hdfs.server.federation.router.RBFConfigKeys.FEDERATION_MOUNT_TABLE_MAX_CACHE_SIZE;
 import static org.apache.hadoop.hdfs.server.federation.router.RBFConfigKeys.FEDERATION_MOUNT_TABLE_MAX_CACHE_SIZE;
 import static org.apache.hadoop.hdfs.server.federation.router.RBFConfigKeys.FEDERATION_MOUNT_TABLE_MAX_CACHE_SIZE_DEFAULT;
 import static org.apache.hadoop.hdfs.server.federation.router.RBFConfigKeys.FEDERATION_MOUNT_TABLE_MAX_CACHE_SIZE_DEFAULT;
+import static org.apache.hadoop.hdfs.server.federation.router.RBFConfigKeys.FEDERATION_MOUNT_TABLE_CACHE_ENABLE;
+import static org.apache.hadoop.hdfs.server.federation.router.RBFConfigKeys.FEDERATION_MOUNT_TABLE_CACHE_ENABLE_DEFAULT;
 import static org.apache.hadoop.hdfs.server.federation.router.FederationUtil.isParentEntry;
 import static org.apache.hadoop.hdfs.server.federation.router.FederationUtil.isParentEntry;
 
 
 import java.io.IOException;
 import java.io.IOException;
@@ -124,12 +126,19 @@ public class MountTableResolver
       this.stateStore = null;
       this.stateStore = null;
     }
     }
 
 
-    int maxCacheSize = conf.getInt(
-        FEDERATION_MOUNT_TABLE_MAX_CACHE_SIZE,
-        FEDERATION_MOUNT_TABLE_MAX_CACHE_SIZE_DEFAULT);
-    this.locationCache = CacheBuilder.newBuilder()
-        .maximumSize(maxCacheSize)
-        .build();
+    boolean mountTableCacheEnable = conf.getBoolean(
+        FEDERATION_MOUNT_TABLE_CACHE_ENABLE,
+        FEDERATION_MOUNT_TABLE_CACHE_ENABLE_DEFAULT);
+    if (mountTableCacheEnable) {
+      int maxCacheSize = conf.getInt(
+          FEDERATION_MOUNT_TABLE_MAX_CACHE_SIZE,
+          FEDERATION_MOUNT_TABLE_MAX_CACHE_SIZE_DEFAULT);
+      this.locationCache = CacheBuilder.newBuilder()
+          .maximumSize(maxCacheSize)
+          .build();
+    } else {
+      this.locationCache = null;
+    }
 
 
     registerCacheExternal();
     registerCacheExternal();
     initDefaultNameService(conf);
     initDefaultNameService(conf);
@@ -239,7 +248,7 @@ public class MountTableResolver
    */
    */
   private void invalidateLocationCache(final String path) {
   private void invalidateLocationCache(final String path) {
     LOG.debug("Invalidating {} from {}", path, locationCache);
     LOG.debug("Invalidating {} from {}", path, locationCache);
-    if (locationCache.size() == 0) {
+    if (locationCache == null || locationCache.size() == 0) {
       return;
       return;
     }
     }
 
 
@@ -359,7 +368,9 @@ public class MountTableResolver
     LOG.info("Clearing all mount location caches");
     LOG.info("Clearing all mount location caches");
     writeLock.lock();
     writeLock.lock();
     try {
     try {
-      this.locationCache.invalidateAll();
+      if (this.locationCache != null) {
+        this.locationCache.invalidateAll();
+      }
       this.tree.clear();
       this.tree.clear();
     } finally {
     } finally {
       writeLock.unlock();
       writeLock.unlock();
@@ -372,6 +383,9 @@ public class MountTableResolver
     verifyMountTable();
     verifyMountTable();
     readLock.lock();
     readLock.lock();
     try {
     try {
+      if (this.locationCache == null) {
+        return lookupLocation(path);
+      }
       Callable<? extends PathLocation> meh = new Callable<PathLocation>() {
       Callable<? extends PathLocation> meh = new Callable<PathLocation>() {
         @Override
         @Override
         public PathLocation call() throws Exception {
         public PathLocation call() throws Exception {
@@ -603,7 +617,10 @@ public class MountTableResolver
    * Get the size of the cache.
    * Get the size of the cache.
    * @return Size of the cache.
    * @return Size of the cache.
    */
    */
-  protected long getCacheSize() {
-    return this.locationCache.size();
+  protected long getCacheSize() throws IOException{
+    if (this.locationCache != null) {
+      return this.locationCache.size();
+    }
+    throw new IOException("localCache is null");
   }
   }
 }
 }

+ 4 - 0
hadoop-hdfs-project/hadoop-hdfs-rbf/src/main/java/org/apache/hadoop/hdfs/server/federation/router/RBFConfigKeys.java

@@ -186,6 +186,10 @@ public class RBFConfigKeys extends CommonConfigurationKeysPublic {
       FEDERATION_ROUTER_PREFIX + "mount-table.max-cache-size";
       FEDERATION_ROUTER_PREFIX + "mount-table.max-cache-size";
   /** Remove cache entries if we have more than 10k. */
   /** Remove cache entries if we have more than 10k. */
   public static final int FEDERATION_MOUNT_TABLE_MAX_CACHE_SIZE_DEFAULT = 10000;
   public static final int FEDERATION_MOUNT_TABLE_MAX_CACHE_SIZE_DEFAULT = 10000;
+  public static final String FEDERATION_MOUNT_TABLE_CACHE_ENABLE =
+      FEDERATION_ROUTER_PREFIX + "mount-table.cache.enable";
+  public static final boolean FEDERATION_MOUNT_TABLE_CACHE_ENABLE_DEFAULT =
+      true;
 
 
   // HDFS Router-based federation admin
   // HDFS Router-based federation admin
   public static final String DFS_ROUTER_ADMIN_HANDLER_COUNT_KEY =
   public static final String DFS_ROUTER_ADMIN_HANDLER_COUNT_KEY =

+ 9 - 0
hadoop-hdfs-project/hadoop-hdfs-rbf/src/main/resources/hdfs-rbf-default.xml

@@ -393,6 +393,15 @@
     </description>
     </description>
   </property>
   </property>
 
 
+  <property>
+    <name>dfs.federation.router.mount-table.cache.enable</name>
+    <value>true</value>
+    <description>
+      Set to true to enable mount table cache (Path to Remote Location cache).
+      Disabling the cache is recommended when a large amount of unique paths are queried.
+    </description>
+  </property>
+
   <property>
   <property>
     <name>dfs.federation.router.quota.enable</name>
     <name>dfs.federation.router.quota.enable</name>
     <value>false</value>
     <value>false</value>

+ 31 - 0
hadoop-hdfs-project/hadoop-hdfs-rbf/src/test/java/org/apache/hadoop/hdfs/server/federation/resolver/TestMountTableResolver.java

@@ -17,6 +17,7 @@
  */
  */
 package org.apache.hadoop.hdfs.server.federation.resolver;
 package org.apache.hadoop.hdfs.server.federation.resolver;
 
 
+import static org.apache.hadoop.hdfs.server.federation.router.RBFConfigKeys.FEDERATION_MOUNT_TABLE_CACHE_ENABLE;
 import static org.apache.hadoop.hdfs.server.federation.router.RBFConfigKeys.FEDERATION_MOUNT_TABLE_MAX_CACHE_SIZE;
 import static org.apache.hadoop.hdfs.server.federation.router.RBFConfigKeys.FEDERATION_MOUNT_TABLE_MAX_CACHE_SIZE;
 import static org.apache.hadoop.hdfs.server.federation.router.RBFConfigKeys.DFS_ROUTER_DEFAULT_NAMESERVICE;
 import static org.apache.hadoop.hdfs.server.federation.router.RBFConfigKeys.DFS_ROUTER_DEFAULT_NAMESERVICE;
 import static org.junit.Assert.assertEquals;
 import static org.junit.Assert.assertEquals;
@@ -37,6 +38,7 @@ import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.hdfs.server.federation.router.Router;
 import org.apache.hadoop.hdfs.server.federation.router.Router;
 import org.apache.hadoop.hdfs.server.federation.store.MountTableStore;
 import org.apache.hadoop.hdfs.server.federation.store.MountTableStore;
 import org.apache.hadoop.hdfs.server.federation.store.records.MountTable;
 import org.apache.hadoop.hdfs.server.federation.store.records.MountTable;
+import org.apache.hadoop.test.GenericTestUtils;
 import org.junit.Before;
 import org.junit.Before;
 import org.junit.Test;
 import org.junit.Test;
 import org.slf4j.Logger;
 import org.slf4j.Logger;
@@ -472,6 +474,35 @@ public class TestMountTableResolver {
     assertNull(entry2);
     assertNull(entry2);
   }
   }
 
 
+  @Test
+  public void testDisableLocalCache() throws IOException {
+    Configuration conf = new Configuration();
+    // Disable mount table cache
+    conf.setBoolean(FEDERATION_MOUNT_TABLE_CACHE_ENABLE, false);
+    conf.setStrings(DFS_ROUTER_DEFAULT_NAMESERVICE, "0");
+    MountTableResolver tmpMountTable = new MountTableResolver(conf);
+
+    // Root mount point
+    Map<String, String> map = getMountTableEntry("1", "/");
+    tmpMountTable.addEntry(MountTable.newInstance("/", map));
+
+    // /tmp
+    map = getMountTableEntry("2", "/tmp");
+    tmpMountTable.addEntry(MountTable.newInstance("/tmp", map));
+
+    // Check localCache is null
+    try {
+      tmpMountTable.getCacheSize();
+      fail("getCacheSize call should fail.");
+    } catch (IOException e) {
+      GenericTestUtils.assertExceptionContains("localCache is null", e);
+    }
+
+    // Check resolve path without cache
+    assertEquals("2->/tmp/tesfile1.txt",
+        tmpMountTable.getDestinationForPath("/tmp/tesfile1.txt").toString());
+  }
+
   @Test
   @Test
   public void testCacheCleaning() throws Exception {
   public void testCacheCleaning() throws Exception {
     for (int i = 0; i < 1000; i++) {
     for (int i = 0; i < 1000; i++) {