Browse Source

HDFS-14731. [FGL] Remove redundant locking on NameNode. Contributed by Konstantin V Shvachko.

(cherry picked from commit ecbcb058b8bc0fbc3903acb56814c6d9608bc396)
Konstantin V Shvachko 5 years ago
parent
commit
4af188587f

+ 6 - 1
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/BackupImage.java

@@ -218,7 +218,12 @@ public class BackupImage extends FSImage {
       }
       lastAppliedTxId = logLoader.getLastAppliedTxId();
 
-      getNamesystem().dir.updateCountForQuota();
+      getNamesystem().writeLock();
+      try {
+        getNamesystem().dir.updateCountForQuota();
+      } finally {
+        getNamesystem().writeUnlock();
+      }
     } finally {
       backupInputStream.clear();
     }

+ 4 - 4
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/EncryptionZoneManager.java

@@ -187,11 +187,11 @@ public class EncryptionZoneManager {
       final int count) throws IOException {
     INodesInPath iip;
     final FSPermissionChecker pc = dir.getPermissionChecker();
-    dir.readLock();
+    dir.getFSNamesystem().readLock();
     try {
       iip = dir.resolvePath(pc, zone, DirOp.READ);
     } finally {
-      dir.readUnlock();
+      dir.getFSNamesystem().readUnlock();
     }
     reencryptionHandler
         .pauseForTestingAfterNthCheckpoint(iip.getLastINode().getId(), count);
@@ -280,11 +280,11 @@ public class EncryptionZoneManager {
     if (getProvider() == null || reencryptionHandler == null) {
       return;
     }
-    dir.writeLock();
+    dir.getFSNamesystem().writeLock();
     try {
       reencryptionHandler.stopThreads();
     } finally {
-      dir.writeUnlock();
+      dir.getFSNamesystem().writeUnlock();
     }
     if (reencryptHandlerExecutor != null) {
       reencryptHandlerExecutor.shutdownNow();

+ 2 - 3
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/FSDirEncryptionZoneOp.java

@@ -382,7 +382,6 @@ final class FSDirEncryptionZoneOp {
   static void saveFileXAttrsForBatch(FSDirectory fsd,
       List<FileEdekInfo> batch) {
     assert fsd.getFSNamesystem().hasWriteLock();
-    assert !fsd.hasWriteLock();
     if (batch != null && !batch.isEmpty()) {
       for (FileEdekInfo entry : batch) {
         final INode inode = fsd.getInode(entry.getInodeId());
@@ -727,13 +726,13 @@ final class FSDirEncryptionZoneOp {
       final FSPermissionChecker pc, final String zone) throws IOException {
     assert dir.getProvider() != null;
     final INodesInPath iip;
-    dir.readLock();
+    dir.getFSNamesystem().readLock();
     try {
       iip = dir.resolvePath(pc, zone, DirOp.READ);
       dir.ezManager.checkEncryptionZoneRoot(iip.getLastINode(), zone);
       return dir.ezManager.getKeyName(iip);
     } finally {
-      dir.readUnlock();
+      dir.getFSNamesystem().readUnlock();
     }
   }
 }

+ 17 - 20
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/FSDirectory.java

@@ -81,7 +81,6 @@ import java.util.SortedSet;
 import java.util.TreeSet;
 import java.util.concurrent.ForkJoinPool;
 import java.util.concurrent.RecursiveAction;
-import java.util.concurrent.locks.ReentrantReadWriteLock;
 
 import static org.apache.hadoop.fs.CommonConfigurationKeys.FS_PROTECTED_DIRECTORIES;
 import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_NAMENODE_ACCESSTIME_PRECISION_DEFAULT;
@@ -170,9 +169,6 @@ public class FSDirectory implements Closeable {
   // Each entry in this set must be a normalized path.
   private volatile SortedSet<String> protectedDirectories;
 
-  // lock to protect the directory and BlockMap
-  private final ReentrantReadWriteLock dirLock;
-
   private final boolean isPermissionEnabled;
   /**
    * Support for ACLs is controlled by a configuration flag. If the
@@ -212,37 +208,44 @@ public class FSDirectory implements Closeable {
     attributeProvider = provider;
   }
 
-  // utility methods to acquire and release read lock and write lock
+  /**
+   * The directory lock dirLock provided redundant locking.
+   * It has been used whenever namesystem.fsLock was used.
+   * dirLock is now removed and utility methods to acquire and release dirLock
+   * remain as placeholders only
+   */
   void readLock() {
-    this.dirLock.readLock().lock();
+    assert namesystem.hasReadLock() : "Should hold namesystem read lock";
   }
 
   void readUnlock() {
-    this.dirLock.readLock().unlock();
+    assert namesystem.hasReadLock() : "Should hold namesystem read lock";
   }
 
   void writeLock() {
-    this.dirLock.writeLock().lock();
+    assert namesystem.hasWriteLock() : "Should hold namesystem write lock";
   }
 
   void writeUnlock() {
-    this.dirLock.writeLock().unlock();
+    assert namesystem.hasWriteLock() : "Should hold namesystem write lock";
   }
 
   boolean hasWriteLock() {
-    return this.dirLock.isWriteLockedByCurrentThread();
+    return namesystem.hasWriteLock();
   }
 
   boolean hasReadLock() {
-    return this.dirLock.getReadHoldCount() > 0 || hasWriteLock();
+    return namesystem.hasReadLock();
   }
 
+  @Deprecated // dirLock is obsolete, use namesystem.fsLock instead
   public int getReadHoldCount() {
-    return this.dirLock.getReadHoldCount();
+    return namesystem.getReadHoldCount();
   }
 
+  @Deprecated // dirLock is obsolete, use namesystem.fsLock instead
   public int getWriteHoldCount() {
-    return this.dirLock.getWriteHoldCount();
+    return namesystem.getWriteHoldCount();
   }
 
   @VisibleForTesting
@@ -268,7 +271,6 @@ public class FSDirectory implements Closeable {
   FSDirectory(FSNamesystem ns, Configuration conf) throws IOException {
     // used to enable/disable the use of expanded string tables.
     SerialNumberManager.initialize(conf);
-    this.dirLock = new ReentrantReadWriteLock(true); // fair
     this.inodeId = new INodeId();
     rootDir = createRoot(ns);
     inodeMap = INodeMap.newInstance(rootDir);
@@ -1466,12 +1468,7 @@ public class FSDirectory implements Closeable {
    * @return The inode associated with the given id
    */
   public INode getInode(long id) {
-    readLock();
-    try {
-      return inodeMap.get(id);
-    } finally {
-      readUnlock();
-    }
+    return inodeMap.get(id);
   }
   
   @VisibleForTesting

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

@@ -3604,6 +3604,7 @@ public class FSNamesystem implements Namesystem, FSNamesystemMBean,
 
   @Override
   public INodeFile getBlockCollection(long id) {
+    assert hasReadLock() : "Accessing INode id = " + id + " without read lock";
     INode inode = getFSDirectory().getInode(id);
     return inode == null ? null : inode.asFile();
   }

+ 2 - 2
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/ReencryptionHandler.java

@@ -337,7 +337,7 @@ public class ReencryptionHandler implements Runnable {
       }
 
       final Long zoneId;
-      dir.readLock();
+      dir.getFSNamesystem().readLock();
       try {
         zoneId = getReencryptionStatus().getNextUnprocessedZone();
         if (zoneId == null) {
@@ -349,7 +349,7 @@ public class ReencryptionHandler implements Runnable {
         getReencryptionStatus().markZoneStarted(zoneId);
         resetSubmissionTracker(zoneId);
       } finally {
-        dir.readUnlock();
+        dir.getFSNamesystem().readUnlock();
       }
 
       try {

+ 14 - 2
hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/server/blockmanagement/TestBlocksWithNotEnoughRacks.java

@@ -503,7 +503,8 @@ public class TestBlocksWithNotEnoughRacks {
       BlockInfo storedBlock = bm.getStoredBlock(b.getLocalBlock());
 
       // The block should be replicated OK - so Reconstruction Work will be null
-      BlockReconstructionWork work = bm.scheduleReconstruction(storedBlock, 2);
+      BlockReconstructionWork work = scheduleReconstruction(
+          cluster.getNamesystem(), storedBlock, 2);
       assertNull(work);
       // Set the upgradeDomain to "3" for the 3 nodes hosting the block.
       // Then alternately set the remaining 3 nodes to have an upgradeDomain
@@ -519,7 +520,8 @@ public class TestBlocksWithNotEnoughRacks {
         }
       }
       // Now reconWork is non-null and 2 extra targets are needed
-      work = bm.scheduleReconstruction(storedBlock, 2);
+      work = scheduleReconstruction(
+          cluster.getNamesystem(), storedBlock, 2);
       assertEquals(2, work.getAdditionalReplRequired());
 
       // Add the block to the replication queue and ensure it is replicated
@@ -531,6 +533,16 @@ public class TestBlocksWithNotEnoughRacks {
     }
   }
 
+  static BlockReconstructionWork scheduleReconstruction(
+      FSNamesystem fsn, BlockInfo block, int priority) {
+    fsn.writeLock();
+    try {
+      return fsn.getBlockManager().scheduleReconstruction(block, priority);
+    } finally {
+      fsn.writeUnlock();
+    }
+  }
+
   @Test
   public void testUnderReplicatedRespectsRacksAndUpgradeDomain()
       throws Exception {