Переглянути джерело

HDFS-17405. [FGL] Using different metric name to trace performance for FGL and Global lock (#6600)

ZanderXu 1 рік тому
батько
коміт
f0368bb237

+ 19 - 15
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/FSNamesystemLock.java

@@ -56,7 +56,7 @@ import static org.apache.hadoop.log.LogThrottlingHelper.LogAction;
  * to be true, metrics will be emitted into the FSNamesystem metrics registry
  * to be true, metrics will be emitted into the FSNamesystem metrics registry
  * for each operation which acquires this lock indicating how long the operation
  * for each operation which acquires this lock indicating how long the operation
  * held the lock for. These metrics have names of the form
  * held the lock for. These metrics have names of the form
- * FSN(Read|Write)LockNanosOperationName, where OperationName denotes the name
+ * ${LockName}(Read|Write)LockNanosOperationName, where OperationName denotes the name
  * of the operation that initiated the lock hold (this will be OTHER for certain
  * of the operation that initiated the lock hold (this will be OTHER for certain
  * uncategorized operations) and they export the hold time values in
  * uncategorized operations) and they export the hold time values in
  * nanoseconds. Note that if a thread dies, metrics produced after the
  * nanoseconds. Note that if a thread dies, metrics produced after the
@@ -67,6 +67,7 @@ import static org.apache.hadoop.log.LogThrottlingHelper.LogAction;
 public class FSNamesystemLock {
 public class FSNamesystemLock {
   @VisibleForTesting
   @VisibleForTesting
   protected ReentrantReadWriteLock coarseLock;
   protected ReentrantReadWriteLock coarseLock;
+  private final String lockName;
 
 
   private volatile boolean metricsEnabled;
   private volatile boolean metricsEnabled;
   private final MutableRatesWithAggregation detailedHoldTimeMetrics;
   private final MutableRatesWithAggregation detailedHoldTimeMetrics;
@@ -123,23 +124,26 @@ public class FSNamesystemLock {
 
 
   @VisibleForTesting
   @VisibleForTesting
   static final String OP_NAME_OTHER = "OTHER";
   static final String OP_NAME_OTHER = "OTHER";
-  private static final String READ_LOCK_METRIC_PREFIX = "FSNReadLock";
-  private static final String WRITE_LOCK_METRIC_PREFIX = "FSNWriteLock";
+  private final String readLockMetricPrefix;
+  private final String writeLockMetricPrefix;
   private static final String LOCK_METRIC_SUFFIX = "Nanos";
   private static final String LOCK_METRIC_SUFFIX = "Nanos";
 
 
   private static final String OVERALL_METRIC_NAME = "Overall";
   private static final String OVERALL_METRIC_NAME = "Overall";
 
 
-  public FSNamesystemLock(Configuration conf,
+  public FSNamesystemLock(Configuration conf, String lockName,
       MutableRatesWithAggregation detailedHoldTimeMetrics) {
       MutableRatesWithAggregation detailedHoldTimeMetrics) {
-    this(conf, detailedHoldTimeMetrics, new Timer());
+    this(conf, lockName, detailedHoldTimeMetrics, new Timer());
   }
   }
 
 
   @VisibleForTesting
   @VisibleForTesting
-  FSNamesystemLock(Configuration conf,
+  FSNamesystemLock(Configuration conf, String lockName,
       MutableRatesWithAggregation detailedHoldTimeMetrics, Timer timer) {
       MutableRatesWithAggregation detailedHoldTimeMetrics, Timer timer) {
+    this.lockName = lockName;
+    this.readLockMetricPrefix = this.lockName + "ReadLock";
+    this.writeLockMetricPrefix = this.lockName + "WriteLock";
     boolean fair = conf.getBoolean(DFS_NAMENODE_FSLOCK_FAIR_KEY,
     boolean fair = conf.getBoolean(DFS_NAMENODE_FSLOCK_FAIR_KEY,
         DFS_NAMENODE_FSLOCK_FAIR_DEFAULT);
         DFS_NAMENODE_FSLOCK_FAIR_DEFAULT);
-    FSNamesystem.LOG.info("fsLock is fair: " + fair);
+    FSNamesystem.LOG.info("{}Lock is fair: {}.", this.lockName, fair);
     this.coarseLock = new ReentrantReadWriteLock(fair);
     this.coarseLock = new ReentrantReadWriteLock(fair);
     this.timer = timer;
     this.timer = timer;
 
 
@@ -157,8 +161,8 @@ public class FSNamesystemLock {
     this.metricsEnabled = conf.getBoolean(
     this.metricsEnabled = conf.getBoolean(
         DFS_NAMENODE_LOCK_DETAILED_METRICS_KEY,
         DFS_NAMENODE_LOCK_DETAILED_METRICS_KEY,
         DFS_NAMENODE_LOCK_DETAILED_METRICS_DEFAULT);
         DFS_NAMENODE_LOCK_DETAILED_METRICS_DEFAULT);
-    FSNamesystem.LOG.info("Detailed lock hold time metrics enabled: " +
-        this.metricsEnabled);
+    FSNamesystem.LOG.info("Detailed lock hold time metrics of {}Lock is {}.",
+        this.lockName, this.metricsEnabled ? "enabled" : "disabled");
     this.detailedHoldTimeMetrics = detailedHoldTimeMetrics;
     this.detailedHoldTimeMetrics = detailedHoldTimeMetrics;
   }
   }
 
 
@@ -231,9 +235,9 @@ public class FSNamesystemLock {
       LockHeldInfo lockHeldInfo =
       LockHeldInfo lockHeldInfo =
           longestReadLockHeldInfo.getAndSet(new LockHeldInfo());
           longestReadLockHeldInfo.getAndSet(new LockHeldInfo());
       FSNamesystem.LOG.info(
       FSNamesystem.LOG.info(
-          "\tNumber of suppressed read-lock reports: {}"
+          "\tNumber of suppressed read-lock reports of {}Lock is {}"
               + "\n\tLongest read-lock held at {} for {}ms by {}{} via {}",
               + "\n\tLongest read-lock held at {} for {}ms by {}{} via {}",
-          numSuppressedWarnings, Time.formatTime(lockHeldInfo.getStartTimeMs()),
+          this.lockName, numSuppressedWarnings, Time.formatTime(lockHeldInfo.getStartTimeMs()),
           lockHeldInfo.getIntervalMs(), lockHeldInfo.getOpName(),
           lockHeldInfo.getIntervalMs(), lockHeldInfo.getOpName(),
           lockHeldInfo.getLockReportInfo(), lockHeldInfo.getStackTrace());
           lockHeldInfo.getLockReportInfo(), lockHeldInfo.getStackTrace());
     }
     }
@@ -337,10 +341,10 @@ public class FSNamesystemLock {
 
 
     if (logAction.shouldLog()) {
     if (logAction.shouldLog()) {
       FSNamesystem.LOG.info(
       FSNamesystem.LOG.info(
-          "\tNumber of suppressed write-lock reports: {}"
+          "\tNumber of suppressed write-lock reports of {}Lock is {}"
               + "\n\tLongest write-lock held at {} for {}ms by {}{} via {}"
               + "\n\tLongest write-lock held at {} for {}ms by {}{} via {}"
               + "\n\tTotal suppressed write-lock held time: {}",
               + "\n\tTotal suppressed write-lock held time: {}",
-          logAction.getCount() - 1,
+          this.lockName, logAction.getCount() - 1,
           Time.formatTime(lockHeldInfo.getStartTimeMs()),
           Time.formatTime(lockHeldInfo.getStartTimeMs()),
           lockHeldInfo.getIntervalMs(), lockHeldInfo.getOpName(),
           lockHeldInfo.getIntervalMs(), lockHeldInfo.getOpName(),
           lockHeldInfo.getLockReportInfo(), lockHeldInfo.getStackTrace(),
           lockHeldInfo.getLockReportInfo(), lockHeldInfo.getStackTrace(),
@@ -456,8 +460,8 @@ public class FSNamesystemLock {
     }
     }
   }
   }
 
 
-  private static String getMetricName(String operationName, boolean isWrite) {
-    return (isWrite ? WRITE_LOCK_METRIC_PREFIX : READ_LOCK_METRIC_PREFIX) +
+  private String getMetricName(String operationName, boolean isWrite) {
+    return (isWrite ? this.writeLockMetricPrefix : this.readLockMetricPrefix) +
         org.apache.commons.lang3.StringUtils.capitalize(operationName) +
         org.apache.commons.lang3.StringUtils.capitalize(operationName) +
         LOCK_METRIC_SUFFIX;
         LOCK_METRIC_SUFFIX;
   }
   }

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

@@ -35,8 +35,8 @@ public class FineGrainedFSNamesystemLock implements FSNLockManager {
   private final FSNamesystemLock bmLock;
   private final FSNamesystemLock bmLock;
 
 
   public FineGrainedFSNamesystemLock(Configuration conf, MutableRatesWithAggregation aggregation) {
   public FineGrainedFSNamesystemLock(Configuration conf, MutableRatesWithAggregation aggregation) {
-    this.fsLock = new FSNamesystemLock(conf, aggregation);
-    this.bmLock = new FSNamesystemLock(conf, aggregation);
+    this.fsLock = new FSNamesystemLock(conf, "FS", aggregation);
+    this.bmLock = new FSNamesystemLock(conf, "BM", aggregation);
   }
   }
 
 
   @Override
   @Override

+ 1 - 1
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/fgl/GlobalFSNamesystemLock.java

@@ -29,7 +29,7 @@ public class GlobalFSNamesystemLock implements FSNLockManager {
   private final FSNamesystemLock lock;
   private final FSNamesystemLock lock;
 
 
   public GlobalFSNamesystemLock(Configuration conf, MutableRatesWithAggregation aggregation) {
   public GlobalFSNamesystemLock(Configuration conf, MutableRatesWithAggregation aggregation) {
-    this.lock = new FSNamesystemLock(conf, aggregation);
+    this.lock = new FSNamesystemLock(conf, "FSN", aggregation);
   }
   }
 
 
   @Override
   @Override

+ 10 - 10
hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/server/namenode/TestFSNamesystemLock.java

@@ -57,17 +57,17 @@ public class TestFSNamesystemLock {
     Configuration conf = new Configuration();
     Configuration conf = new Configuration();
 
 
     conf.setBoolean(DFS_NAMENODE_FSLOCK_FAIR_KEY, true);
     conf.setBoolean(DFS_NAMENODE_FSLOCK_FAIR_KEY, true);
-    FSNamesystemLock fsnLock = new FSNamesystemLock(conf, null);
+    FSNamesystemLock fsnLock = new FSNamesystemLock(conf, "FSN", null);
     assertTrue(fsnLock.coarseLock.isFair());
     assertTrue(fsnLock.coarseLock.isFair());
 
 
     conf.setBoolean(DFS_NAMENODE_FSLOCK_FAIR_KEY, false);
     conf.setBoolean(DFS_NAMENODE_FSLOCK_FAIR_KEY, false);
-    fsnLock = new FSNamesystemLock(conf, null);
+    fsnLock = new FSNamesystemLock(conf, "FSN", null);
     assertFalse(fsnLock.coarseLock.isFair());
     assertFalse(fsnLock.coarseLock.isFair());
   }
   }
 
 
   @Test
   @Test
   public void testFSNamesystemLockCompatibility() {
   public void testFSNamesystemLockCompatibility() {
-    FSNamesystemLock rwLock = new FSNamesystemLock(new Configuration(), null);
+    FSNamesystemLock rwLock = new FSNamesystemLock(new Configuration(), "FSN", null);
 
 
     assertEquals(0, rwLock.getReadHoldCount());
     assertEquals(0, rwLock.getReadHoldCount());
     rwLock.readLock();
     rwLock.readLock();
@@ -107,7 +107,7 @@ public class TestFSNamesystemLock {
     final CountDownLatch latch = new CountDownLatch(threadCount);
     final CountDownLatch latch = new CountDownLatch(threadCount);
     final Configuration conf = new Configuration();
     final Configuration conf = new Configuration();
     conf.setBoolean(DFS_NAMENODE_FSLOCK_FAIR_KEY, true);
     conf.setBoolean(DFS_NAMENODE_FSLOCK_FAIR_KEY, true);
-    final FSNamesystemLock rwLock = new FSNamesystemLock(conf, null);
+    final FSNamesystemLock rwLock = new FSNamesystemLock(conf, "FSN", null);
     rwLock.writeLock();
     rwLock.writeLock();
     ExecutorService helper = Executors.newFixedThreadPool(threadCount);
     ExecutorService helper = Executors.newFixedThreadPool(threadCount);
 
 
@@ -150,7 +150,7 @@ public class TestFSNamesystemLock {
         writeLockSuppressWarningInterval, TimeUnit.MILLISECONDS);
         writeLockSuppressWarningInterval, TimeUnit.MILLISECONDS);
 
 
     final FakeTimer timer = new FakeTimer();
     final FakeTimer timer = new FakeTimer();
-    final FSNamesystemLock fsnLock = new FSNamesystemLock(conf, null, timer);
+    final FSNamesystemLock fsnLock = new FSNamesystemLock(conf, "FSN", null, timer);
     timer.advance(writeLockSuppressWarningInterval);
     timer.advance(writeLockSuppressWarningInterval);
 
 
     LogCapturer logs = LogCapturer.captureLogs(FSNamesystem.LOG);
     LogCapturer logs = LogCapturer.captureLogs(FSNamesystem.LOG);
@@ -213,7 +213,7 @@ public class TestFSNamesystemLock {
         "held at " + Time.formatTime(timer.now()).substring(0, 10);
         "held at " + Time.formatTime(timer.now()).substring(0, 10);
     assertTrue(logs.getOutput().contains(startTimeStr));
     assertTrue(logs.getOutput().contains(startTimeStr));
     assertTrue(logs.getOutput().contains(
     assertTrue(logs.getOutput().contains(
-        "Number of suppressed write-lock reports: 2"));
+        "Number of suppressed write-lock reports of FSNLock is 2"));
   }
   }
 
 
   /**
   /**
@@ -233,7 +233,7 @@ public class TestFSNamesystemLock {
         readLockSuppressWarningInterval, TimeUnit.MILLISECONDS);
         readLockSuppressWarningInterval, TimeUnit.MILLISECONDS);
 
 
     final FakeTimer timer = new FakeTimer();
     final FakeTimer timer = new FakeTimer();
-    final FSNamesystemLock fsnLock = new FSNamesystemLock(conf, null, timer);
+    final FSNamesystemLock fsnLock = new FSNamesystemLock(conf, "FSN", null, timer);
     timer.advance(readLockSuppressWarningInterval);
     timer.advance(readLockSuppressWarningInterval);
 
 
     LogCapturer logs = LogCapturer.captureLogs(FSNamesystem.LOG);
     LogCapturer logs = LogCapturer.captureLogs(FSNamesystem.LOG);
@@ -304,7 +304,7 @@ public class TestFSNamesystemLock {
         "held at " + Time.formatTime(timer.now()).substring(0, 10);
         "held at " + Time.formatTime(timer.now()).substring(0, 10);
     assertTrue(logs.getOutput().contains(startTimeStr));
     assertTrue(logs.getOutput().contains(startTimeStr));
     assertTrue(logs.getOutput().contains(
     assertTrue(logs.getOutput().contains(
-        "Number of suppressed read-lock reports: 3"));
+        "Number of suppressed read-lock reports of FSNLock is 3"));
 
 
     // Report if it's held for a long time (and time since last report
     // Report if it's held for a long time (and time since last report
     // exceeds the suppress warning interval) while another thread also has the
     // exceeds the suppress warning interval) while another thread also has the
@@ -366,7 +366,7 @@ public class TestFSNamesystemLock {
     MetricsRegistry registry = new MetricsRegistry("Test");
     MetricsRegistry registry = new MetricsRegistry("Test");
     MutableRatesWithAggregation rates =
     MutableRatesWithAggregation rates =
         registry.newRatesWithAggregation("Test");
         registry.newRatesWithAggregation("Test");
-    FSNamesystemLock fsLock = new FSNamesystemLock(conf, rates, timer);
+    FSNamesystemLock fsLock = new FSNamesystemLock(conf, "FSN", rates, timer);
 
 
     fsLock.readLock();
     fsLock.readLock();
     timer.advanceNanos(1300000);
     timer.advanceNanos(1300000);
@@ -419,7 +419,7 @@ public class TestFSNamesystemLock {
         writeLockSuppressWarningInterval, TimeUnit.MILLISECONDS);
         writeLockSuppressWarningInterval, TimeUnit.MILLISECONDS);
 
 
     final FakeTimer timer = new FakeTimer();
     final FakeTimer timer = new FakeTimer();
-    final FSNamesystemLock fsnLock = new FSNamesystemLock(conf, null, timer);
+    final FSNamesystemLock fsnLock = new FSNamesystemLock(conf, "FSN", null, timer);
     timer.advance(writeLockSuppressWarningInterval);
     timer.advance(writeLockSuppressWarningInterval);
 
 
     LogCapturer logs = LogCapturer.captureLogs(FSNamesystem.LOG);
     LogCapturer logs = LogCapturer.captureLogs(FSNamesystem.LOG);