浏览代码

HDFS-4182. SecondaryNameNode leaks NameCache entries (bobby)

git-svn-id: https://svn.apache.org/repos/asf/hadoop/common/branches/branch-0.23@1409479 13f79535-47bb-0310-9956-ffa450edef68
Robert Joseph Evans 12 年之前
父节点
当前提交
11b92b1da9

+ 2 - 0
hadoop-hdfs-project/hadoop-hdfs/CHANGES.txt

@@ -54,6 +54,8 @@ Release 0.23.5 - UNRELEASED
     HDFS-4172. namenode does not URI-encode parameters when building URI for
     datanode request (Derek Dagit via bobby)
 
+    HDFS-4182. SecondaryNameNode leaks NameCache entries (bobby)
+
 Release 0.23.4 - UNRELEASED
 
   INCOMPATIBLE CHANGES

+ 35 - 3
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/FSDirectory.java

@@ -60,6 +60,9 @@ import org.apache.hadoop.hdfs.server.common.HdfsServerConstants.BlockUCState;
 import org.apache.hadoop.hdfs.server.common.HdfsServerConstants.StartupOption;
 import org.apache.hadoop.hdfs.util.ByteArray;
 
+import com.google.common.annotations.VisibleForTesting;
+import com.google.common.base.Preconditions;
+
 /*************************************************
  * FSDirectory stores the filesystem directory state.
  * It handles writing/loading values to disk, and logging
@@ -184,6 +187,21 @@ public class FSDirectory implements Closeable {
         fsImage.close();
       }
     }
+    setReady();
+  }
+  
+
+  /**
+   * Notify that loading of this FSDirectory is complete, and
+   * it is ready for use 
+   */
+  void imageLoadComplete() {
+    Preconditions.checkState(!ready, "FSDirectory already loaded");
+    setReady();
+  }
+
+  void setReady() {
+    if(ready) return;
     writeLock();
     try {
       setReady(true);
@@ -193,6 +211,12 @@ public class FSDirectory implements Closeable {
       writeUnlock();
     }
   }
+  
+  //This is for testing purposes only
+  @VisibleForTesting
+  boolean isReady() {
+    return ready;
+  }
 
   // exposed for unit tests
   protected void setReady(boolean flag) {
@@ -2129,9 +2153,16 @@ public class FSDirectory implements Closeable {
    * Reset the entire namespace tree.
    */
   void reset() {
-    rootDir = new INodeDirectoryWithQuota(INodeDirectory.ROOT_NAME,
-        getFSNamesystem().createFsOwnerPermissions(new FsPermission((short)0755)),
-        Integer.MAX_VALUE, -1);
+    writeLock();
+    try {
+      setReady(false);
+      rootDir = new INodeDirectoryWithQuota(INodeDirectory.ROOT_NAME,
+          getFSNamesystem().createFsOwnerPermissions(new FsPermission((short)0755)),
+          Integer.MAX_VALUE, -1);
+      nameCache.reset();
+    } finally {
+      writeUnlock();
+    }
   }
 
   /**
@@ -2293,4 +2324,5 @@ public class FSDirectory implements Closeable {
       inode.setLocalName(name.getBytes());
     }
   }
+
 }

+ 10 - 0
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/NameCache.java

@@ -152,4 +152,14 @@ class NameCache<K> {
     cache.put(name, name);
     lookups += useThreshold;
   }
+
+  public void reset() {
+    initialized = false;
+    cache.clear();
+    if (transientMap == null) {
+      transientMap = new HashMap<K, UseCount>();
+    } else {
+      transientMap.clear();
+    }
+  }
 }

+ 1 - 0
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/SecondaryNameNode.java

@@ -859,6 +859,7 @@ public class SecondaryNameNode implements Runnable {
             "just been downloaded");
       }
       dstImage.reloadFromImageFile(file);
+      dstImage.getFSNamesystem().dir.imageLoadComplete();
     }
     
     Checkpointer.rollForwardByApplyingLogs(manifest, dstImage);

+ 11 - 0
hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/server/namenode/TestNameCache.java

@@ -55,6 +55,17 @@ public class TestNameCache {
     for (String s : notMatching) {
       verifyNameReuse(cache, s, false);
     }
+    
+    cache.reset();
+    cache.initialized();
+    
+    for (String s : matching) {
+      verifyNameReuse(cache, s, false);
+    }
+    
+    for (String s : notMatching) {
+      verifyNameReuse(cache, s, false);
+    }
   }
 
   private void verifyNameReuse(NameCache<String> cache, String s, boolean reused) {