Browse Source

HDFS-10838. Last full block report received time for each DN should be easily discoverable. Contributed by Surendra Singh Lilhore.

Arpit Agarwal 8 years ago
parent
commit
8dc8ecbb91

+ 50 - 2
hadoop-hdfs-project/hadoop-hdfs-client/src/main/java/org/apache/hadoop/hdfs/protocol/DatanodeInfo.java

@@ -85,6 +85,8 @@ public class DatanodeInfo extends DatanodeID implements Node {
 
   protected AdminStates adminState;
   private long maintenanceExpireTimeInMS;
+  private long lastBlockReportTime;
+  private long lastBlockReportMonotonic;
 
   protected DatanodeInfo(DatanodeInfo from) {
     super(from);
@@ -101,6 +103,8 @@ public class DatanodeInfo extends DatanodeID implements Node {
     this.location = from.getNetworkLocation();
     this.adminState = from.getAdminState();
     this.upgradeDomain = from.getUpgradeDomain();
+    this.lastBlockReportTime = from.getLastBlockReportTime();
+    this.lastBlockReportMonotonic = from.getLastBlockReportMonotonic();
   }
 
   protected DatanodeInfo(DatanodeID nodeID) {
@@ -116,6 +120,8 @@ public class DatanodeInfo extends DatanodeID implements Node {
     this.lastUpdateMonotonic = 0L;
     this.xceiverCount = 0;
     this.adminState = null;
+    this.lastBlockReportTime = 0L;
+    this.lastBlockReportMonotonic = 0L;
   }
 
   protected DatanodeInfo(DatanodeID nodeID, String location) {
@@ -131,7 +137,8 @@ public class DatanodeInfo extends DatanodeID implements Node {
       final long blockPoolUsed, final long cacheCapacity, final long cacheUsed,
       final long lastUpdate, final long lastUpdateMonotonic,
       final int xceiverCount, final String networkLocation,
-      final AdminStates adminState, final String upgradeDomain) {
+      final AdminStates adminState, final String upgradeDomain,
+      final long lastBlockReportTime, final long lastBlockReportMonotonic) {
     super(ipAddr, hostName, datanodeUuid, xferPort, infoPort, infoSecurePort,
         ipcPort);
     this.capacity = capacity;
@@ -147,6 +154,8 @@ public class DatanodeInfo extends DatanodeID implements Node {
     this.location = networkLocation;
     this.adminState = adminState;
     this.upgradeDomain = upgradeDomain;
+    this.lastBlockReportTime = lastBlockReportTime;
+    this.lastBlockReportMonotonic = lastBlockReportMonotonic;
   }
 
   /** Network location name. */
@@ -391,6 +400,11 @@ public class DatanodeInfo extends DatanodeID implements Node {
         .append(percent2String(cacheRemainingPercent)).append("\n");
     buffer.append("Xceivers: ").append(getXceiverCount()).append("\n");
     buffer.append("Last contact: ").append(new Date(lastUpdate)).append("\n");
+    buffer
+        .append("Last Block Report: ")
+        .append(
+            lastBlockReportTime != 0 ? new Date(lastBlockReportTime) : "Never")
+        .append("\n");
     return buffer.toString();
   }
 
@@ -503,6 +517,26 @@ public class DatanodeInfo extends DatanodeID implements Node {
     return this.maintenanceExpireTimeInMS;
   }
 
+  /** Sets the last block report time. */
+  public void setLastBlockReportTime(long lastBlockReportTime) {
+    this.lastBlockReportTime = lastBlockReportTime;
+  }
+
+  /** Sets the last block report monotonic time. */
+  public void setLastBlockReportMonotonic(long lastBlockReportMonotonic) {
+    this.lastBlockReportMonotonic = lastBlockReportMonotonic;
+  }
+
+  /** Last block report time. */
+  public long getLastBlockReportTime() {
+    return lastBlockReportTime;
+  }
+
+  /** Last block report monotonic time. */
+  public long getLastBlockReportMonotonic() {
+    return lastBlockReportMonotonic;
+  }
+
   /**
    * Take the node out of maintenance mode.
    */
@@ -643,6 +677,8 @@ public class DatanodeInfo extends DatanodeID implements Node {
     private int infoSecurePort;
     private int ipcPort;
     private long nonDfsUsed = 0L;
+    private long lastBlockReportTime = 0L;
+    private long lastBlockReportMonotonic = 0L;
 
     public DatanodeInfoBuilder setFrom(DatanodeInfo from) {
       this.capacity = from.getCapacity();
@@ -658,6 +694,8 @@ public class DatanodeInfo extends DatanodeID implements Node {
       this.location = from.getNetworkLocation();
       this.adminState = from.getAdminState();
       this.upgradeDomain = from.getUpgradeDomain();
+      this.lastBlockReportTime = from.getLastBlockReportTime();
+      this.lastBlockReportMonotonic = from.getLastBlockReportMonotonic();
       setNodeID(from);
       return this;
     }
@@ -775,12 +813,22 @@ public class DatanodeInfo extends DatanodeID implements Node {
       return this;
     }
 
+    public DatanodeInfoBuilder setLastBlockReportTime(long time) {
+      this.lastBlockReportTime = time;
+      return this;
+    }
+
+    public DatanodeInfoBuilder setLastBlockReportMonotonic(long time) {
+      this.lastBlockReportMonotonic = time;
+      return this;
+    }
+
     public DatanodeInfo build() {
       return new DatanodeInfo(ipAddr, hostName, datanodeUuid, xferPort,
           infoPort, infoSecurePort, ipcPort, capacity, dfsUsed, nonDfsUsed,
           remaining, blockPoolUsed, cacheCapacity, cacheUsed, lastUpdate,
           lastUpdateMonotonic, xceiverCount, location, adminState,
-          upgradeDomain);
+          upgradeDomain, lastBlockReportTime, lastBlockReportMonotonic);
     }
   }
 }

+ 7 - 1
hadoop-hdfs-project/hadoop-hdfs-client/src/main/java/org/apache/hadoop/hdfs/protocolPB/PBHelperClient.java

@@ -294,6 +294,8 @@ public class PBHelperClient {
         .setLastUpdateMonotonic(info.getLastUpdateMonotonic())
         .setXceiverCount(info.getXceiverCount())
         .setAdminState(convert(info.getAdminState()))
+        .setLastBlockReportTime(info.getLastBlockReportTime())
+        .setLastBlockReportMonotonic(info.getLastBlockReportMonotonic())
         .build();
     return builder.build();
   }
@@ -563,7 +565,11 @@ public class PBHelperClient {
             .setLastUpdateMonotonic(di.getLastUpdateMonotonic())
             .setXceiverCount(di.getXceiverCount())
             .setAdminState(convert(di.getAdminState())).setUpgradeDomain(
-            di.hasUpgradeDomain() ? di.getUpgradeDomain() : null);
+            di.hasUpgradeDomain() ? di.getUpgradeDomain() : null)
+            .setLastBlockReportTime(di.hasLastBlockReportTime() ?
+                di.getLastBlockReportTime() : 0)
+            .setLastBlockReportMonotonic(di.hasLastBlockReportMonotonic() ?
+                di.getLastBlockReportMonotonic() : 0);
     if (di.hasNonDfsUsed()) {
       dinfo.setNonDfsUsed(di.getNonDfsUsed());
     } else {

+ 2 - 0
hadoop-hdfs-project/hadoop-hdfs-client/src/main/java/org/apache/hadoop/hdfs/web/JsonUtilClient.java

@@ -295,6 +295,8 @@ class JsonUtilClient {
             DatanodeInfo.AdminStates
                 .valueOf(getString(m, "adminState", "NORMAL")))
         .setUpgradeDomain(getString(m, "upgradeDomain", ""))
+        .setLastBlockReportTime(getLong(m, "lastBlockReportTime", 0L))
+        .setLastBlockReportMonotonic(getLong(m, "lastBlockReportMonotonic", 0L))
         .build();
   }
 

+ 2 - 0
hadoop-hdfs-project/hadoop-hdfs-client/src/main/proto/hdfs.proto

@@ -102,6 +102,8 @@ message DatanodeInfoProto {
   optional uint64 cacheUsed = 12 [default = 0];
   optional uint64 lastUpdateMonotonic = 13 [default = 0];
   optional string upgradeDomain = 14;
+  optional uint64 lastBlockReportTime = 15 [default = 0];
+  optional uint64 lastBlockReportMonotonic = 16 [default = 0];
 }
 
 /**

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

@@ -18,6 +18,7 @@
 package org.apache.hadoop.hdfs.server.blockmanagement;
 
 import static org.apache.hadoop.util.ExitUtil.terminate;
+import static org.apache.hadoop.util.Time.now;
 
 import java.io.IOException;
 import java.io.PrintWriter;
@@ -2158,6 +2159,8 @@ public class BlockManager implements BlockStatsMXBean {
           long leaseId = this.getBlockReportLeaseManager().removeLease(node);
           BlockManagerFaultInjector.getInstance().
               removeBlockReportLease(node, leaseId);
+          node.setLastBlockReportTime(now());
+          node.setLastBlockReportMonotonic(Time.monotonicNow());
         }
         LOG.debug("Processing RPC with index {} out of total {} RPCs in "
                 + "processReport 0x{}", context.getCurRpc(),

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

@@ -5445,7 +5445,9 @@ public class FSNamesystem implements Namesystem, FSNamesystemMBean,
           .put("blockScheduled", node.getBlocksScheduled())
           .put("blockPoolUsed", node.getBlockPoolUsed())
           .put("blockPoolUsedPercent", node.getBlockPoolUsedPercent())
-          .put("volfails", node.getVolumeFailures());
+          .put("volfails", node.getVolumeFailures())
+          // Block report time in minutes
+          .put("lastBlockReport", getLastBlockReport(node));
       VolumeFailureSummary volumeFailureSummary = node.getVolumeFailureSummary();
       if (volumeFailureSummary != null) {
         innerinfo
@@ -5544,6 +5546,10 @@ public class FSNamesystem implements Namesystem, FSNamesystemMBean,
     return (monotonicNow() - alivenode.getLastUpdateMonotonic())/1000;
   }
 
+  private Object getLastBlockReport(DatanodeDescriptor node) {
+    return (monotonicNow() - node.getLastBlockReportMonotonic()) / 60000;
+  }
+
   private long getDfsUsed(DatanodeDescriptor alivenode) {
     return alivenode.getDfsUsed();
   }

+ 3 - 0
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/web/JsonUtil.java

@@ -181,6 +181,9 @@ public class JsonUtil {
     if (datanodeinfo.getUpgradeDomain() != null) {
       m.put("upgradeDomain", datanodeinfo.getUpgradeDomain());
     }
+    m.put("lastBlockReportTime", datanodeinfo.getLastBlockReportTime());
+    m.put("lastBlockReportMonotonic",
+        datanodeinfo.getLastBlockReportMonotonic());
     return m;
   }
 

+ 2 - 0
hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/hdfs/dfshealth.html

@@ -309,6 +309,7 @@
       <th>Node</th>
       <th>Http Address</th>
       <th>Last contact</th>
+      <th>Last Block Report</th>
       <th style="width:180px; text-align:center">Capacity</th>
       <th>Blocks</th>
       <th>Block pool used</th>
@@ -320,6 +321,7 @@
     <td ng-value="{state}-{name}" class="dfshealth-node-icon dfshealth-node-{state}">{name} ({xferaddr})</td>
     <td ng-value="{state}-{name}"><a href='//{dnWebAddress}'>{dnWebAddress}</a></td>
     <td ng-value="{lastContact}">{lastContact}s</td>
+    <td ng-value="{lastBlockReport}">{lastBlockReport}m</td>
     <td ng-value="{usedPercentage}">
       <div>
         <div style="display:inline-block; float: left; padding-right: 10px;">{capacity|fmt_bytes}</div>

+ 1 - 0
hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/hdfs/dfshealth.js

@@ -333,6 +333,7 @@
               { 'orderDataType': 'ng-value', 'searchable': true },
               { 'orderDataType': 'ng-value', 'type': 'numeric' },
               { 'orderDataType': 'ng-value', 'type': 'numeric' },
+              { 'orderDataType': 'ng-value', 'type': 'numeric' },
               { 'orderData': 3, 'type': 'numeric' },
               { 'orderDataType': 'ng-value', 'type': 'numeric'},
               { 'orderData': 5 }

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

@@ -161,6 +161,7 @@ public class TestNameNodeMXBean {
         assertTrue(((Long)liveNode.get("capacity")) > 0);
         assertTrue(liveNode.containsKey("numBlocks"));
         assertTrue(((Long)liveNode.get("numBlocks")) == 0);
+        assertTrue(liveNode.containsKey("lastBlockReport"));
         // a. By default the upgrade domain isn't defined on any DN.
         // b. If the upgrade domain is set on a DN, JMX should have the same
         // value.