浏览代码

HDFS-16352. return the real datanode numBlocks in #getDatanodeStorageReport (#3714). Contributed by liubingxing.

Signed-off-by: He Xiaoqiao <hexiaoqiao@apache.org>
liubingxing 3 年之前
父节点
当前提交
d8dea6f52a

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

@@ -698,9 +698,10 @@ public class DatanodeInfo extends DatanodeID implements Node {
     private long nonDfsUsed = 0L;
     private long lastBlockReportTime = 0L;
     private long lastBlockReportMonotonic = 0L;
-    private int numBlocks;
-
+    private int numBlocks = 0;
 
+    // Please use setNumBlocks explicitly to set numBlocks as this method doesn't have
+    // sufficient info about numBlocks
     public DatanodeInfoBuilder setFrom(DatanodeInfo from) {
       this.capacity = from.getCapacity();
       this.dfsUsed = from.getDfsUsed();
@@ -717,7 +718,6 @@ public class DatanodeInfo extends DatanodeID implements Node {
       this.upgradeDomain = from.getUpgradeDomain();
       this.lastBlockReportTime = from.getLastBlockReportTime();
       this.lastBlockReportMonotonic = from.getLastBlockReportMonotonic();
-      this.numBlocks = from.getNumBlocks();
       setNodeID(from);
       return this;
     }

+ 2 - 1
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/blockmanagement/DatanodeManager.java

@@ -2182,7 +2182,8 @@ public class DatanodeManager {
     for (int i = 0; i < reports.length; i++) {
       final DatanodeDescriptor d = datanodes.get(i);
       reports[i] = new DatanodeStorageReport(
-          new DatanodeInfoBuilder().setFrom(d).build(), d.getStorageReports());
+          new DatanodeInfoBuilder().setFrom(d).setNumBlocks(d.numBlocks()).build(),
+          d.getStorageReports());
     }
     return reports;
   }

+ 30 - 0
hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/server/namenode/TestNameNodeRpcServerMethods.java

@@ -20,9 +20,14 @@ package org.apache.hadoop.hdfs.server.namenode;
 import java.io.IOException;
 
 import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.fs.FSDataOutputStream;
+import org.apache.hadoop.fs.Path;
 import org.apache.hadoop.fs.UnresolvedLinkException;
+import org.apache.hadoop.hdfs.DistributedFileSystem;
 import org.apache.hadoop.hdfs.HdfsConfiguration;
 import org.apache.hadoop.hdfs.MiniDFSCluster;
+import org.apache.hadoop.hdfs.protocol.HdfsConstants;
+import org.apache.hadoop.hdfs.server.protocol.DatanodeStorageReport;
 import org.apache.hadoop.hdfs.server.protocol.NamenodeProtocols;
 import org.apache.hadoop.security.AccessControlException;
 import org.apache.hadoop.test.GenericTestUtils;
@@ -31,6 +36,8 @@ import org.junit.Assert;
 import org.junit.Before;
 import org.junit.Test;
 
+import static org.junit.Assert.assertEquals;
+
 public class TestNameNodeRpcServerMethods {
   private static NamenodeProtocols nnRpc;
   private static Configuration conf;
@@ -83,4 +90,27 @@ public class TestNameNodeRpcServerMethods {
 
   }
 
+  @Test
+  public void testGetDatanodeStorageReportWithNumBLocksNotZero() throws Exception {
+    int buffSize = 1024;
+    long blockSize = 1024 * 1024;
+    String file = "/testFile";
+    DistributedFileSystem dfs = cluster.getFileSystem();
+    FSDataOutputStream outputStream = dfs.create(
+        new Path(file), true, buffSize, (short)1, blockSize);
+    byte[] outBuffer = new byte[buffSize];
+    for (int i = 0; i < buffSize; i++) {
+      outBuffer[i] = (byte) (i & 0x00ff);
+    }
+    outputStream.write(outBuffer);
+    outputStream.close();
+
+    int numBlocks = 0;
+    DatanodeStorageReport[] reports
+        = nnRpc.getDatanodeStorageReport(HdfsConstants.DatanodeReportType.ALL);
+    for (DatanodeStorageReport r : reports) {
+      numBlocks += r.getDatanodeInfo().getNumBlocks();
+    }
+    assertEquals(1, numBlocks);
+  }
 }