Explorar o código

HDFS-443. Add a new metrics numExpiredHeartbeats to the Namenode. Contributed by Jitendra Nath Pandey

git-svn-id: https://svn.apache.org/repos/asf/hadoop/hdfs/trunk@794007 13f79535-47bb-0310-9956-ffa450edef68
Tsz-wo Sze %!s(int64=16) %!d(string=hai) anos
pai
achega
b010deca84

+ 3 - 0
CHANGES.txt

@@ -36,6 +36,9 @@ Trunk (unreleased changes)
 
 
     HDFS-278. HDFS Outputstream close does not hang forever. (dhruba)
     HDFS-278. HDFS Outputstream close does not hang forever. (dhruba)
 
 
+    HDFS-443. Add a new metrics numExpiredHeartbeats to the Namenode.
+    (Jitendra Nath Pandey via szetszwo)
+
   BUG FIXES
   BUG FIXES
     HDFS-76. Better error message to users when commands fail because of 
     HDFS-76. Better error message to users when commands fail because of 
     lack of quota. Allow quota to be set even if the limit is lower than
     lack of quota. Allow quota to be set even if the limit is lower than

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

@@ -2225,6 +2225,7 @@ public class FSNamesystem implements FSConstants, FSNamesystemMBean {
              it.hasNext();) {
              it.hasNext();) {
           DatanodeDescriptor nodeInfo = it.next();
           DatanodeDescriptor nodeInfo = it.next();
           if (isDatanodeDead(nodeInfo)) {
           if (isDatanodeDead(nodeInfo)) {
+            myFSMetrics.numExpiredHeartbeats.inc();
             foundDead = true;
             foundDead = true;
             nodeID = nodeInfo;
             nodeID = nodeInfo;
             break;
             break;

+ 3 - 0
src/java/org/apache/hadoop/hdfs/server/namenode/metrics/FSNamesystemMetrics.java

@@ -27,6 +27,7 @@ import org.apache.hadoop.metrics.util.MetricsBase;
 import org.apache.hadoop.metrics.util.MetricsIntValue;
 import org.apache.hadoop.metrics.util.MetricsIntValue;
 import org.apache.hadoop.metrics.util.MetricsLongValue;
 import org.apache.hadoop.metrics.util.MetricsLongValue;
 import org.apache.hadoop.metrics.util.MetricsRegistry;
 import org.apache.hadoop.metrics.util.MetricsRegistry;
+import org.apache.hadoop.metrics.util.MetricsTimeVaryingInt;
 
 
 /**
 /**
  * 
  * 
@@ -59,6 +60,8 @@ public class FSNamesystemMetrics implements Updater {
   final MetricsIntValue scheduledReplicationBlocks = new MetricsIntValue("ScheduledReplicationBlocks", registry);
   final MetricsIntValue scheduledReplicationBlocks = new MetricsIntValue("ScheduledReplicationBlocks", registry);
   final MetricsIntValue missingBlocks = new MetricsIntValue("MissingBlocks", registry);    
   final MetricsIntValue missingBlocks = new MetricsIntValue("MissingBlocks", registry);    
   final MetricsIntValue blockCapacity = new MetricsIntValue("BlockCapacity", registry);
   final MetricsIntValue blockCapacity = new MetricsIntValue("BlockCapacity", registry);
+  public final MetricsTimeVaryingInt numExpiredHeartbeats = 
+                          new MetricsTimeVaryingInt("ExpiredHeartbeats", registry);
 
 
   private final FSNamesystem fsNameSystem;
   private final FSNamesystem fsNameSystem;
 
 

+ 7 - 0
src/test/hdfs/org/apache/hadoop/hdfs/TestDatanodeReport.java

@@ -26,6 +26,7 @@ import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.hdfs.protocol.DatanodeInfo;
 import org.apache.hadoop.hdfs.protocol.DatanodeInfo;
 import org.apache.hadoop.hdfs.protocol.FSConstants.DatanodeReportType;
 import org.apache.hadoop.hdfs.protocol.FSConstants.DatanodeReportType;
 import org.apache.hadoop.hdfs.server.datanode.DataNode;
 import org.apache.hadoop.hdfs.server.datanode.DataNode;
+import org.apache.hadoop.hdfs.server.namenode.metrics.FSNamesystemMetrics;
 
 
 /**
 /**
  * This test ensures the all types of data node report work correctly.
  * This test ensures the all types of data node report work correctly.
@@ -40,6 +41,7 @@ public class TestDatanodeReport extends TestCase {
   public void testDatanodeReport() throws Exception {
   public void testDatanodeReport() throws Exception {
     conf.setInt(
     conf.setInt(
         "heartbeat.recheck.interval", 500); // 0.5s
         "heartbeat.recheck.interval", 500); // 0.5s
+    conf.setLong("dfs.heartbeat.interval", 1L);
     MiniDFSCluster cluster = 
     MiniDFSCluster cluster = 
       new MiniDFSCluster(conf, NUM_OF_DATANODES, true, null);
       new MiniDFSCluster(conf, NUM_OF_DATANODES, true, null);
     try {
     try {
@@ -73,6 +75,11 @@ public class TestDatanodeReport extends TestCase {
                    NUM_OF_DATANODES-1);
                    NUM_OF_DATANODES-1);
       assertEquals(client.datanodeReport(DatanodeReportType.ALL).length,
       assertEquals(client.datanodeReport(DatanodeReportType.ALL).length,
                    NUM_OF_DATANODES);
                    NUM_OF_DATANODES);
+
+      Thread.sleep(5000);
+      FSNamesystemMetrics fsMetrics = 
+                     cluster.getNameNode().getNamesystem().getFSNamesystemMetrics();
+      assertEquals(1,fsMetrics.numExpiredHeartbeats.getCurrentIntervalValue());
     }finally {
     }finally {
       cluster.shutdown();
       cluster.shutdown();
     }
     }