Browse Source

HDFS-12206. Rename the split EC / replicated block metrics.

Andrew Wang 8 years ago
parent
commit
480c8db40c

+ 1 - 1
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/DFSConfigKeys.java

@@ -1031,7 +1031,7 @@ public class DFSConfigKeys extends CommonConfigurationKeys {
   public static final boolean DFS_PIPELINE_ECN_ENABLED_DEFAULT = false;
 
   // Key Provider Cache Expiry
-  public static final String DFS_DATANODE_BLOCK_PINNING_ENABLED = 
+  public static final String DFS_DATANODE_BLOCK_PINNING_ENABLED =
     "dfs.datanode.block-pinning.enabled";
   public static final boolean DFS_DATANODE_BLOCK_PINNING_ENABLED_DEFAULT =
     false;

+ 11 - 11
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/blockmanagement/BlockManager.java

@@ -232,47 +232,47 @@ public class BlockManager implements BlockStatsMXBean {
   }
 
   /** Used by metrics. */
-  public long getLowRedundancyBlocksStat() {
+  public long getLowRedundancyBlocks() {
     return neededReconstruction.getLowRedundancyBlocksStat();
   }
 
   /** Used by metrics. */
-  public long getCorruptBlocksStat() {
+  public long getCorruptBlocks() {
     return corruptReplicas.getCorruptBlocksStat();
   }
 
   /** Used by metrics. */
-  public long getMissingBlocksStat() {
+  public long getMissingBlocks() {
     return neededReconstruction.getCorruptBlocksStat();
   }
 
   /** Used by metrics. */
-  public long getMissingReplicationOneBlocksStat() {
+  public long getMissingReplicationOneBlocks() {
     return neededReconstruction.getCorruptReplicationOneBlocksStat();
   }
 
   /** Used by metrics. */
-  public long getPendingDeletionBlocksStat() {
+  public long getPendingDeletionReplicatedBlocks() {
     return invalidateBlocks.getBlocksStat();
   }
 
   /** Used by metrics. */
-  public long getLowRedundancyECBlockGroupsStat() {
+  public long getLowRedundancyECBlockGroups() {
     return neededReconstruction.getLowRedundancyECBlockGroupsStat();
   }
 
   /** Used by metrics. */
-  public long getCorruptECBlockGroupsStat() {
+  public long getCorruptECBlockGroups() {
     return corruptReplicas.getCorruptECBlockGroupsStat();
   }
 
   /** Used by metrics. */
-  public long getMissingECBlockGroupsStat() {
+  public long getMissingECBlockGroups() {
     return neededReconstruction.getCorruptECBlockGroupsStat();
   }
 
   /** Used by metrics. */
-  public long getPendingDeletionECBlockGroupsStat() {
+  public long getPendingDeletionECBlockGroups() {
     return invalidateBlocks.getECBlockGroupsStat();
   }
 
@@ -2292,11 +2292,11 @@ public class BlockManager implements BlockStatsMXBean {
     return bmSafeMode.getBytesInFuture();
   }
 
-  public long getBytesInFutureReplicatedBlocksStat() {
+  public long getBytesInFutureReplicatedBlocks() {
     return bmSafeMode.getBytesInFutureBlocks();
   }
 
-  public long getBytesInFutureStripedBlocksStat() {
+  public long getBytesInFutureECBlockGroups() {
     return bmSafeMode.getBytesInFutureECBlockGroups();
   }
 

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

@@ -92,6 +92,7 @@ import static org.apache.hadoop.hdfs.server.namenode.FSDirStatAndListingOp.*;
 import org.apache.hadoop.hdfs.protocol.BlocksStats;
 import org.apache.hadoop.hdfs.protocol.ECBlockGroupsStats;
 import org.apache.hadoop.hdfs.protocol.OpenFileEntry;
+import org.apache.hadoop.hdfs.server.namenode.metrics.ReplicatedBlocksMBean;
 import org.apache.hadoop.hdfs.server.protocol.SlowDiskReports;
 import static org.apache.hadoop.util.Time.now;
 import static org.apache.hadoop.util.Time.monotonicNow;
@@ -243,10 +244,9 @@ import org.apache.hadoop.hdfs.server.namenode.NameNodeLayoutVersion.Feature;
 import org.apache.hadoop.hdfs.server.namenode.ha.EditLogTailer;
 import org.apache.hadoop.hdfs.server.namenode.ha.HAContext;
 import org.apache.hadoop.hdfs.server.namenode.ha.StandbyCheckpointer;
-import org.apache.hadoop.hdfs.server.namenode.metrics.ECBlockGroupsStatsMBean;
+import org.apache.hadoop.hdfs.server.namenode.metrics.ECBlockGroupsMBean;
 import org.apache.hadoop.hdfs.server.namenode.metrics.FSNamesystemMBean;
 import org.apache.hadoop.hdfs.server.namenode.metrics.NameNodeMetrics;
-import org.apache.hadoop.hdfs.server.namenode.metrics.ReplicatedBlocksStatsMBean;
 import org.apache.hadoop.hdfs.server.namenode.snapshot.DirectorySnapshottableFeature;
 import org.apache.hadoop.hdfs.server.namenode.snapshot.Snapshot;
 import org.apache.hadoop.hdfs.server.namenode.snapshot.SnapshotManager;
@@ -340,7 +340,7 @@ import com.google.common.util.concurrent.ThreadFactoryBuilder;
 @InterfaceAudience.Private
 @Metrics(context="dfs")
 public class FSNamesystem implements Namesystem, FSNamesystemMBean,
-    NameNodeMXBean, ReplicatedBlocksStatsMBean, ECBlockGroupsStatsMBean {
+    NameNodeMXBean, ReplicatedBlocksMBean, ECBlockGroupsMBean {
   public static final Log LOG = LogFactory.getLog(FSNamesystem.class);
   private final MetricsRegistry registry = new MetricsRegistry("FSNamesystem");
   @Metric final MutableRatesWithAggregation detailedLockHoldTimeMetrics =
@@ -4076,10 +4076,10 @@ public class FSNamesystem implements Namesystem, FSNamesystemMBean,
    * @see ClientProtocol#getBlocksStats()
    */
   BlocksStats getBlocksStats() {
-    return new BlocksStats(getLowRedundancyBlocksStat(),
-        getCorruptBlocksStat(), getMissingBlocksStat(),
-        getMissingReplicationOneBlocksStat(), getBlocksBytesInFutureStat(),
-        getPendingDeletionBlocksStat());
+    return new BlocksStats(getLowRedundancyReplicatedBlocks(),
+        getCorruptReplicatedBlocks(), getMissingReplicatedBlocks(),
+        getMissingReplicationOneBlocks(), getBytesInFutureReplicatedBlocks(),
+        getPendingDeletionReplicatedBlocks());
   }
 
   /**
@@ -4089,9 +4089,9 @@ public class FSNamesystem implements Namesystem, FSNamesystemMBean,
    * @see ClientProtocol#getECBlockGroupsStats()
    */
   ECBlockGroupsStats getECBlockGroupsStats() {
-    return new ECBlockGroupsStats(getLowRedundancyECBlockGroupsStat(),
-        getCorruptECBlockGroupsStat(), getMissingECBlockGroupsStat(),
-        getECBlocksBytesInFutureStat(), getPendingDeletionECBlockGroupsStat());
+    return new ECBlockGroupsStats(getLowRedundancyECBlockGroups(),
+        getCorruptECBlockGroups(), getMissingECBlockGroups(),
+        getBytesInFutureECBlockGroups(), getPendingDeletionECBlockGroups());
   }
 
   @Override // FSNamesystemMBean
@@ -4638,76 +4638,76 @@ public class FSNamesystem implements Namesystem, FSNamesystemMBean,
   @Override // ReplicatedBlocksMBean
   @Metric({"LowRedundancyReplicatedBlocks",
       "Number of low redundancy replicated blocks"})
-  public long getLowRedundancyBlocksStat() {
-    return blockManager.getLowRedundancyBlocksStat();
+  public long getLowRedundancyReplicatedBlocks() {
+    return blockManager.getLowRedundancyBlocks();
   }
 
   @Override // ReplicatedBlocksMBean
   @Metric({"CorruptReplicatedBlocks", "Number of corrupted replicated blocks"})
-  public long getCorruptBlocksStat() {
-    return blockManager.getCorruptBlocksStat();
+  public long getCorruptReplicatedBlocks() {
+    return blockManager.getCorruptBlocks();
   }
 
   @Override // ReplicatedBlocksMBean
   @Metric({"MissingReplicatedBlocks", "Number of missing replicated blocks"})
-  public long getMissingBlocksStat() {
-    return blockManager.getMissingBlocksStat();
+  public long getMissingReplicatedBlocks() {
+    return blockManager.getMissingBlocks();
   }
 
   @Override // ReplicatedBlocksMBean
-  @Metric({"MissingReplicatedOneBlocks", "Number of missing replicated blocks" +
-      " with replication factor 1"})
-  public long getMissingReplicationOneBlocksStat() {
-    return blockManager.getMissingReplicationOneBlocksStat();
+  @Metric({"MissingReplicationOneBlocks", "Number of missing replicated " +
+      "blocks with replication factor 1"})
+  public long getMissingReplicationOneBlocks() {
+    return blockManager.getMissingReplicationOneBlocks();
   }
 
   @Override // ReplicatedBlocksMBean
-  @Metric({"BytesReplicatedFutureBlocks", "Total bytes in replicated blocks " +
-      "with future generation stamp"})
-  public long getBlocksBytesInFutureStat() {
-    return blockManager.getBytesInFutureReplicatedBlocksStat();
+  @Metric({"BytesInFutureReplicatedBlocks", "Total bytes in replicated " +
+      "blocks with future generation stamp"})
+  public long getBytesInFutureReplicatedBlocks() {
+    return blockManager.getBytesInFutureReplicatedBlocks();
   }
 
   @Override // ReplicatedBlocksMBean
   @Metric({"PendingDeletionReplicatedBlocks", "Number of replicated blocks " +
       "that are pending deletion"})
-  public long getPendingDeletionBlocksStat() {
-    return blockManager.getPendingDeletionBlocksStat();
+  public long getPendingDeletionReplicatedBlocks() {
+    return blockManager.getPendingDeletionReplicatedBlocks();
   }
 
-  @Override // ECBlockGroupsStatsMBean
+  @Override // ECBlockGroupsMBean
   @Metric({"LowRedundancyECBlockGroups", "Number of erasure coded block " +
       "groups with low redundancy"})
-  public long getLowRedundancyECBlockGroupsStat() {
-    return blockManager.getLowRedundancyECBlockGroupsStat();
+  public long getLowRedundancyECBlockGroups() {
+    return blockManager.getLowRedundancyECBlockGroups();
   }
 
-  @Override // ECBlockGroupsStatsMBean
+  @Override // ECBlockGroupsMBean
   @Metric({"CorruptECBlockGroups", "Number of erasure coded block groups that" +
       " are corrupt"})
-  public long getCorruptECBlockGroupsStat() {
-    return blockManager.getCorruptECBlockGroupsStat();
+  public long getCorruptECBlockGroups() {
+    return blockManager.getCorruptECBlockGroups();
   }
 
-  @Override // ECBlockGroupsStatsMBean
+  @Override // ECBlockGroupsMBean
   @Metric({"MissingECBlockGroups", "Number of erasure coded block groups that" +
       " are missing"})
-  public long getMissingECBlockGroupsStat() {
-    return blockManager.getMissingECBlockGroupsStat();
+  public long getMissingECBlockGroups() {
+    return blockManager.getMissingECBlockGroups();
   }
 
-  @Override // ECBlockGroupsStatsMBean
-  @Metric({"BytesFutureECBlockGroups", "Total bytes in erasure coded block " +
+  @Override // ECBlockGroupsMBean
+  @Metric({"BytesInFutureECBlockGroups", "Total bytes in erasure coded block " +
       "groups with future generation stamp"})
-  public long getECBlocksBytesInFutureStat() {
-    return blockManager.getBytesInFutureStripedBlocksStat();
+  public long getBytesInFutureECBlockGroups() {
+    return blockManager.getBytesInFutureECBlockGroups();
   }
 
-  @Override // ECBlockGroupsStatsMBean
+  @Override // ECBlockGroupsMBean
   @Metric({"PendingDeletionECBlockGroups", "Number of erasure coded block " +
       "groups that are pending deletion"})
-  public long getPendingDeletionECBlockGroupsStat() {
-    return blockManager.getPendingDeletionECBlockGroupsStat();
+  public long getPendingDeletionECBlockGroups() {
+    return blockManager.getPendingDeletionECBlockGroups();
   }
 
   @Override
@@ -4774,9 +4774,9 @@ public class FSNamesystem implements Namesystem, FSNamesystemMBean,
    * Register following MBeans with their respective names.
    * FSNamesystemMBean:
    *        "hadoop:service=NameNode,name=FSNamesystemState"
-   * ReplicatedBlocksStatsMBean:
+   * ReplicatedBlocksMBean:
    *        "hadoop:service=NameNode,name=ReplicatedBlocksState"
-   * ECBlockGroupsStatsMBean:
+   * ECBlockGroupsMBean:
    *        "hadoop:service=NameNode,name=ECBlockGroupsState"
    */
   private void registerMBean() {
@@ -4785,9 +4785,9 @@ public class FSNamesystem implements Namesystem, FSNamesystemMBean,
       StandardMBean namesystemBean = new StandardMBean(
           this, FSNamesystemMBean.class);
       StandardMBean replicaBean = new StandardMBean(
-          this, ReplicatedBlocksStatsMBean.class);
+          this, ReplicatedBlocksMBean.class);
       StandardMBean ecBean = new StandardMBean(
-          this, ECBlockGroupsStatsMBean.class);
+          this, ECBlockGroupsMBean.class);
       namesystemMBeanName = MBeans.register(
           "NameNode", "FSNamesystemState", namesystemBean);
       replicatedBlocksMBeanName = MBeans.register(

+ 6 - 6
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/metrics/ECBlockGroupsStatsMBean.java → hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/metrics/ECBlockGroupsMBean.java

@@ -31,29 +31,29 @@ import org.apache.hadoop.classification.InterfaceAudience;
  *
  */
 @InterfaceAudience.Private
-public interface ECBlockGroupsStatsMBean {
+public interface ECBlockGroupsMBean {
   /**
    * Return count of erasure coded block groups with low redundancy.
    */
-  long getLowRedundancyECBlockGroupsStat();
+  long getLowRedundancyECBlockGroups();
 
   /**
    * Return count of erasure coded block groups that are corrupt.
    */
-  long getCorruptECBlockGroupsStat();
+  long getCorruptECBlockGroups();
 
   /**
    * Return count of erasure coded block groups that are missing.
    */
-  long getMissingECBlockGroupsStat();
+  long getMissingECBlockGroups();
 
   /**
    * Return total bytes of erasure coded future block groups.
    */
-  long getECBlocksBytesInFutureStat();
+  long getBytesInFutureECBlockGroups();
 
   /**
    * Return count of erasure coded block groups that are pending deletion.
    */
-  long getPendingDeletionECBlockGroupsStat();
+  long getPendingDeletionECBlockGroups();
 }

+ 7 - 7
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/metrics/ReplicatedBlocksStatsMBean.java → hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/metrics/ReplicatedBlocksMBean.java

@@ -30,34 +30,34 @@ import org.apache.hadoop.classification.InterfaceAudience;
  * @see org.apache.hadoop.hdfs.server.namenode.metrics.NameNodeMetrics
  */
 @InterfaceAudience.Private
-public interface ReplicatedBlocksStatsMBean {
+public interface ReplicatedBlocksMBean {
   /**
    * Return low redundancy blocks count.
    */
-  long getLowRedundancyBlocksStat();
+  long getLowRedundancyReplicatedBlocks();
 
   /**
    * Return corrupt blocks count.
    */
-  long getCorruptBlocksStat();
+  long getCorruptReplicatedBlocks();
 
   /**
    * Return missing blocks count.
    */
-  long getMissingBlocksStat();
+  long getMissingReplicatedBlocks();
 
   /**
    * Return count of missing blocks with replication factor one.
    */
-  long getMissingReplicationOneBlocksStat();
+  long getMissingReplicationOneBlocks();
 
   /**
    * Return total bytes of future blocks.
    */
-  long getBlocksBytesInFutureStat();
+  long getBytesInFutureReplicatedBlocks();
 
   /**
    * Return count of blocks that are pending deletion.
    */
-  long getPendingDeletionBlocksStat();
+  long getPendingDeletionReplicatedBlocks();
 }

+ 2 - 2
hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/server/namenode/TestAddStripedBlocks.java

@@ -424,9 +424,9 @@ public class TestAddStripedBlocks {
         cluster.getDataNodes().get(3).getDatanodeId(), reports[0]);
     BlockManagerTestUtil.updateState(ns.getBlockManager());
     // the total number of corrupted block info is still 1
-    Assert.assertEquals(1, ns.getCorruptECBlockGroupsStat());
+    Assert.assertEquals(1, ns.getCorruptECBlockGroups());
     Assert.assertEquals(1, ns.getCorruptReplicaBlocks());
-    Assert.assertEquals(0, ns.getCorruptBlocksStat());
+    Assert.assertEquals(0, ns.getCorruptReplicatedBlocks());
     // 2 internal blocks corrupted
     Assert.assertEquals(2, bm.getCorruptReplicas(stored).size());
 

+ 4 - 4
hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/server/namenode/TestNameNodeMXBean.java

@@ -809,10 +809,10 @@ public class TestNameNodeMXBean {
       long totalMissingBlocks = cluster.getNamesystem().getMissingBlocksCount();
       Long replicaMissingBlocks =
           (Long) mbs.getAttribute(replStateMBeanName,
-              "MissingBlocksStat");
+              "MissingReplicatedBlocks");
       Long ecMissingBlocks =
           (Long) mbs.getAttribute(ecBlkGrpStateMBeanName,
-              "MissingECBlockGroupsStat");
+              "MissingECBlockGroups");
       assertEquals("Unexpected total missing blocks!",
           expectedMissingBlockCount, totalMissingBlocks);
       assertEquals("Unexpected total missing blocks!",
@@ -826,10 +826,10 @@ public class TestNameNodeMXBean {
           cluster.getNamesystem().getCorruptReplicaBlocks();
       Long replicaCorruptBlocks =
           (Long) mbs.getAttribute(replStateMBeanName,
-              "CorruptBlocksStat");
+              "CorruptReplicatedBlocks");
       Long ecCorruptBlocks =
           (Long) mbs.getAttribute(ecBlkGrpStateMBeanName,
-              "CorruptECBlockGroupsStat");
+              "CorruptECBlockGroups");
       assertEquals("Unexpected total corrupt blocks!",
           expectedCorruptBlockCount, totalCorruptBlocks);
       assertEquals("Unexpected total corrupt blocks!",

+ 2 - 2
hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/server/namenode/TestReconstructStripedBlocks.java

@@ -419,7 +419,7 @@ public class TestReconstructStripedBlocks {
 
       // Verify low redundancy count matching EC block groups count
       BlockManagerTestUtil.updateState(bm);
-      assertEquals(blockGroups, bm.getLowRedundancyECBlockGroupsStat());
+      assertEquals(blockGroups, bm.getLowRedundancyECBlockGroups());
       DFSTestUtil.verifyClientStats(conf, dfsCluster);
 
 
@@ -429,7 +429,7 @@ public class TestReconstructStripedBlocks {
 
       // Verify pending reconstruction count
       assertEquals(blockGroups, getNumberOfBlocksToBeErasureCoded(dfsCluster));
-      assertEquals(0, bm.getLowRedundancyECBlockGroupsStat());
+      assertEquals(0, bm.getLowRedundancyECBlockGroups());
       DFSTestUtil.verifyClientStats(conf, dfsCluster);
     } finally {
       dfsCluster.shutdown();

+ 11 - 11
hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/server/namenode/metrics/TestNameNodeMetrics.java

@@ -317,27 +317,27 @@ public class TestNameNodeMetrics {
         namesystem.getUnderReplicatedBlocks());
     assertEquals("Low redundancy metrics not matching!",
         namesystem.getLowRedundancyBlocks(),
-        namesystem.getLowRedundancyBlocksStat() +
-            namesystem.getLowRedundancyECBlockGroupsStat());
+        namesystem.getLowRedundancyReplicatedBlocks() +
+            namesystem.getLowRedundancyECBlockGroups());
     assertEquals("Corrupt blocks metrics not matching!",
         namesystem.getCorruptReplicaBlocks(),
-        namesystem.getCorruptBlocksStat() +
-            namesystem.getCorruptECBlockGroupsStat());
+        namesystem.getCorruptReplicatedBlocks() +
+            namesystem.getCorruptECBlockGroups());
     assertEquals("Missing blocks metrics not matching!",
         namesystem.getMissingBlocksCount(),
-        namesystem.getMissingBlocksStat() +
-            namesystem.getMissingECBlockGroupsStat());
+        namesystem.getMissingReplicatedBlocks() +
+            namesystem.getMissingECBlockGroups());
     assertEquals("Missing blocks with replication factor one not matching!",
         namesystem.getMissingReplOneBlocksCount(),
-        namesystem.getMissingReplicationOneBlocksStat());
+        namesystem.getMissingReplicationOneBlocks());
     assertEquals("Bytes in future blocks metrics not matching!",
         namesystem.getBytesInFuture(),
-        namesystem.getBlocksBytesInFutureStat() +
-            namesystem.getECBlocksBytesInFutureStat());
+        namesystem.getBytesInFutureReplicatedBlocks() +
+            namesystem.getBytesInFutureECBlockGroups());
     assertEquals("Pending deletion blocks metrics not matching!",
         namesystem.getPendingDeletionBlocks(),
-        namesystem.getPendingDeletionBlocksStat() +
-            namesystem.getPendingDeletionECBlockGroupsStat());
+        namesystem.getPendingDeletionReplicatedBlocks() +
+            namesystem.getPendingDeletionECBlockGroups());
   }
 
   /** Corrupt a block and ensure metrics reflects it */