Browse Source

HDFS-12573. Divide the total blocks metrics into replicated and erasure coded. Contributed by Takanobu Asanuma.

Manoj Govindassamy 7 years ago
parent
commit
78af6cdc53

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

@@ -256,6 +256,11 @@ public class BlockManager implements BlockStatsMXBean {
     return invalidateBlocks.getBlocks();
     return invalidateBlocks.getBlocks();
   }
   }
 
 
+  /** Used by metrics. */
+  public long getTotalReplicatedBlocks() {
+    return blocksMap.getReplicatedBlocks();
+  }
+
   /** Used by metrics. */
   /** Used by metrics. */
   public long getLowRedundancyECBlockGroups() {
   public long getLowRedundancyECBlockGroups() {
     return neededReconstruction.getLowRedundancyECBlockGroups();
     return neededReconstruction.getLowRedundancyECBlockGroups();
@@ -276,6 +281,11 @@ public class BlockManager implements BlockStatsMXBean {
     return invalidateBlocks.getECBlocks();
     return invalidateBlocks.getECBlocks();
   }
   }
 
 
+  /** Used by metrics. */
+  public long getTotalECBlockGroups() {
+    return blocksMap.getECBlockGroups();
+  }
+
   /**
   /**
    * redundancyRecheckInterval is how often namenode checks for new
    * redundancyRecheckInterval is how often namenode checks for new
    * reconstruction work.
    * reconstruction work.

+ 39 - 1
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/blockmanagement/BlocksMap.java

@@ -19,6 +19,7 @@ package org.apache.hadoop.hdfs.server.blockmanagement;
 
 
 import java.util.Collections;
 import java.util.Collections;
 import java.util.Iterator;
 import java.util.Iterator;
+import java.util.concurrent.atomic.LongAdder;
 
 
 import org.apache.hadoop.hdfs.protocol.Block;
 import org.apache.hadoop.hdfs.protocol.Block;
 import org.apache.hadoop.hdfs.server.namenode.INodeId;
 import org.apache.hadoop.hdfs.server.namenode.INodeId;
@@ -37,6 +38,9 @@ class BlocksMap {
   
   
   private GSet<Block, BlockInfo> blocks;
   private GSet<Block, BlockInfo> blocks;
 
 
+  private final LongAdder totalReplicatedBlocks = new LongAdder();
+  private final LongAdder totalECBlockGroups = new LongAdder();
+
   BlocksMap(int capacity) {
   BlocksMap(int capacity) {
     // Use 2% of total memory to size the GSet capacity
     // Use 2% of total memory to size the GSet capacity
     this.capacity = capacity;
     this.capacity = capacity;
@@ -65,6 +69,8 @@ class BlocksMap {
   void clear() {
   void clear() {
     if (blocks != null) {
     if (blocks != null) {
       blocks.clear();
       blocks.clear();
+      totalReplicatedBlocks.reset();
+      totalECBlockGroups.reset();
     }
     }
   }
   }
 
 
@@ -76,6 +82,7 @@ class BlocksMap {
     if (info != b) {
     if (info != b) {
       info = b;
       info = b;
       blocks.put(info);
       blocks.put(info);
+      incrementBlockStat(info);
     }
     }
     info.setBlockCollectionId(bc.getId());
     info.setBlockCollectionId(bc.getId());
     return info;
     return info;
@@ -88,8 +95,10 @@ class BlocksMap {
    */
    */
   void removeBlock(Block block) {
   void removeBlock(Block block) {
     BlockInfo blockInfo = blocks.remove(block);
     BlockInfo blockInfo = blocks.remove(block);
-    if (blockInfo == null)
+    if (blockInfo == null) {
       return;
       return;
+    }
+    decrementBlockStat(block);
 
 
     assert blockInfo.getBlockCollectionId() == INodeId.INVALID_INODE_ID;
     assert blockInfo.getBlockCollectionId() == INodeId.INVALID_INODE_ID;
     final int size = blockInfo.isStriped() ?
     final int size = blockInfo.isStriped() ?
@@ -166,6 +175,7 @@ class BlocksMap {
     if (info.hasNoStorage()    // no datanodes left
     if (info.hasNoStorage()    // no datanodes left
         && info.isDeleted()) { // does not belong to a file
         && info.isDeleted()) { // does not belong to a file
       blocks.remove(b);  // remove block from the map
       blocks.remove(b);  // remove block from the map
+      decrementBlockStat(b);
     }
     }
     return removed;
     return removed;
   }
   }
@@ -196,4 +206,32 @@ class BlocksMap {
   int getCapacity() {
   int getCapacity() {
     return capacity;
     return capacity;
   }
   }
+
+  private void incrementBlockStat(Block block) {
+    if (BlockIdManager.isStripedBlockID(block.getBlockId())) {
+      totalECBlockGroups.increment();
+    } else {
+      totalReplicatedBlocks.increment();
+    }
+  }
+
+  private void decrementBlockStat(Block block) {
+    if (BlockIdManager.isStripedBlockID(block.getBlockId())) {
+      totalECBlockGroups.decrement();
+      assert totalECBlockGroups.longValue() >= 0 :
+          "Total number of ec block groups should be non-negative";
+    } else {
+      totalReplicatedBlocks.decrement();
+      assert totalReplicatedBlocks.longValue() >= 0 :
+          "Total number of replicated blocks should be non-negative";
+    }
+  }
+
+  long getReplicatedBlocks() {
+    return totalReplicatedBlocks.longValue();
+  }
+
+  long getECBlockGroups() {
+    return totalECBlockGroups.longValue();
+  }
 }
 }

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

@@ -4719,6 +4719,12 @@ public class FSNamesystem implements Namesystem, FSNamesystemMBean,
     return blockManager.getPendingDeletionReplicatedBlocks();
     return blockManager.getPendingDeletionReplicatedBlocks();
   }
   }
 
 
+  @Override // ReplicatedBlocksMBean
+  @Metric({"TotalReplicatedBlocks", "Total number of replicated blocks"})
+  public long getTotalReplicatedBlocks() {
+    return blockManager.getTotalReplicatedBlocks();
+  }
+
   @Override // ECBlockGroupsMBean
   @Override // ECBlockGroupsMBean
   @Metric({"LowRedundancyECBlockGroups", "Number of erasure coded block " +
   @Metric({"LowRedundancyECBlockGroups", "Number of erasure coded block " +
       "groups with low redundancy"})
       "groups with low redundancy"})
@@ -4754,6 +4760,12 @@ public class FSNamesystem implements Namesystem, FSNamesystemMBean,
     return blockManager.getPendingDeletionECBlocks();
     return blockManager.getPendingDeletionECBlocks();
   }
   }
 
 
+  @Override // ECBlockGroupsMBean
+  @Metric({"TotalECBlockGroups", "Total number of erasure coded block groups"})
+  public long getTotalECBlockGroups() {
+    return blockManager.getTotalECBlockGroups();
+  }
+
   @Override
   @Override
   public long getBlockDeletionStartTime() {
   public long getBlockDeletionStartTime() {
     return startTime + blockManager.getStartupDelayBlockDeletionInMs();
     return startTime + blockManager.getStartupDelayBlockDeletionInMs();

+ 5 - 0
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/metrics/ECBlockGroupsMBean.java

@@ -56,4 +56,9 @@ public interface ECBlockGroupsMBean {
    * Return count of erasure coded blocks that are pending deletion.
    * Return count of erasure coded blocks that are pending deletion.
    */
    */
   long getPendingDeletionECBlocks();
   long getPendingDeletionECBlocks();
+
+  /**
+   * Return total number of erasure coded block groups.
+   */
+  long getTotalECBlockGroups();
 }
 }

+ 5 - 0
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/metrics/ReplicatedBlocksMBean.java

@@ -60,4 +60,9 @@ public interface ReplicatedBlocksMBean {
    * Return count of blocks that are pending deletion.
    * Return count of blocks that are pending deletion.
    */
    */
   long getPendingDeletionReplicatedBlocks();
   long getPendingDeletionReplicatedBlocks();
+
+  /**
+   * Return total number of replicated blocks.
+   */
+  long getTotalReplicatedBlocks();
 }
 }

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

@@ -731,7 +731,8 @@ public class TestNameNodeMXBean {
     DistributedFileSystem fs = null;
     DistributedFileSystem fs = null;
     try {
     try {
       Configuration conf = new HdfsConfiguration();
       Configuration conf = new HdfsConfiguration();
-      int dataBlocks = StripedFileTestUtil.getDefaultECPolicy().getNumDataUnits();
+      int dataBlocks = StripedFileTestUtil.getDefaultECPolicy()
+          .getNumDataUnits();
       int parityBlocks =
       int parityBlocks =
           StripedFileTestUtil.getDefaultECPolicy().getNumParityUnits();
           StripedFileTestUtil.getDefaultECPolicy().getNumParityUnits();
       int cellSize = StripedFileTestUtil.getDefaultECPolicy().getCellSize();
       int cellSize = StripedFileTestUtil.getDefaultECPolicy().getCellSize();
@@ -860,4 +861,108 @@ public class TestNameNodeMXBean {
       }
       }
     }
     }
   }
   }
+
+  @Test
+  public void testTotalBlocksMetrics() throws Exception {
+    MiniDFSCluster cluster = null;
+    FSNamesystem namesystem = null;
+    DistributedFileSystem fs = null;
+    try {
+      Configuration conf = new HdfsConfiguration();
+      int dataBlocks = StripedFileTestUtil.getDefaultECPolicy()
+          .getNumDataUnits();
+      int parityBlocks =
+          StripedFileTestUtil.getDefaultECPolicy().getNumParityUnits();
+      int totalSize = dataBlocks + parityBlocks;
+      int cellSize = StripedFileTestUtil.getDefaultECPolicy().getCellSize();
+      int stripesPerBlock = 2;
+      int blockSize = stripesPerBlock * cellSize;
+      conf.setLong(DFSConfigKeys.DFS_BLOCK_SIZE_KEY, blockSize);
+
+      cluster = new MiniDFSCluster.Builder(conf)
+          .numDataNodes(totalSize).build();
+      namesystem = cluster.getNamesystem();
+      fs = cluster.getFileSystem();
+      fs.enableErasureCodingPolicy(
+          StripedFileTestUtil.getDefaultECPolicy().getName());
+      verifyTotalBlocksMetrics(0L, 0L, namesystem.getTotalBlocks());
+
+      // create small file
+      Path replDirPath = new Path("/replicated");
+      Path replFileSmall = new Path(replDirPath, "replfile_small");
+      final short factor = 3;
+      DFSTestUtil.createFile(fs, replFileSmall, blockSize, factor, 0);
+      DFSTestUtil.waitReplication(fs, replFileSmall, factor);
+
+      Path ecDirPath = new Path("/striped");
+      fs.mkdir(ecDirPath, FsPermission.getDirDefault());
+      fs.getClient().setErasureCodingPolicy(ecDirPath.toString(),
+          StripedFileTestUtil.getDefaultECPolicy().getName());
+      Path ecFileSmall = new Path(ecDirPath, "ecfile_small");
+      final int smallLength = cellSize * dataBlocks;
+      final byte[] smallBytes = StripedFileTestUtil.generateBytes(smallLength);
+      DFSTestUtil.writeFile(fs, ecFileSmall, smallBytes);
+      verifyTotalBlocksMetrics(1L, 1L, namesystem.getTotalBlocks());
+
+      // create learge file
+      Path replFileLarge = new Path(replDirPath, "replfile_large");
+      DFSTestUtil.createFile(fs, replFileLarge, 2 * blockSize, factor, 0);
+      DFSTestUtil.waitReplication(fs, replFileLarge, factor);
+
+      Path ecFileLarge = new Path(ecDirPath, "ecfile_large");
+      final int largeLength = blockSize * totalSize + smallLength;
+      final byte[] largeBytes = StripedFileTestUtil.generateBytes(largeLength);
+      DFSTestUtil.writeFile(fs, ecFileLarge, largeBytes);
+      verifyTotalBlocksMetrics(3L, 3L, namesystem.getTotalBlocks());
+
+      // delete replicated files
+      fs.delete(replDirPath, true);
+      verifyTotalBlocksMetrics(0L, 3L, namesystem.getTotalBlocks());
+
+      // delete ec files
+      fs.delete(ecDirPath, true);
+      verifyTotalBlocksMetrics(0L, 0L, namesystem.getTotalBlocks());
+    } finally {
+      if (fs != null) {
+        try {
+          fs.close();
+        } catch (Exception e) {
+          throw e;
+        }
+      }
+      if (namesystem != null) {
+        try {
+          namesystem.close();
+        } catch (Exception e) {
+          throw e;
+        }
+      }
+      if (cluster != null) {
+        cluster.shutdown();
+      }
+    }
+  }
+
+  void verifyTotalBlocksMetrics(long expectedTotalReplicatedBlocks,
+      long expectedTotalECBlockGroups, long actualTotalBlocks)
+      throws Exception {
+    long expectedTotalBlocks = expectedTotalReplicatedBlocks
+        + expectedTotalECBlockGroups;
+    assertEquals("Unexpected total blocks!", expectedTotalBlocks,
+        actualTotalBlocks);
+
+    MBeanServer mbs = ManagementFactory.getPlatformMBeanServer();
+    ObjectName replStateMBeanName = new ObjectName(
+        "Hadoop:service=NameNode,name=ReplicatedBlocksState");
+    ObjectName ecBlkGrpStateMBeanName = new ObjectName(
+        "Hadoop:service=NameNode,name=ECBlockGroupsState");
+    Long totalReplicaBlocks = (Long) mbs.getAttribute(replStateMBeanName,
+        "TotalReplicatedBlocks");
+    Long totalECBlockGroups = (Long) mbs.getAttribute(ecBlkGrpStateMBeanName,
+        "TotalECBlockGroups");
+    assertEquals("Unexpected total replicated blocks!",
+        expectedTotalReplicatedBlocks, totalReplicaBlocks.longValue());
+    assertEquals("Unexpected total ec block groups!",
+        expectedTotalECBlockGroups, totalECBlockGroups.longValue());
+  }
 }
 }