Kaynağa Gözat

HADOOP-2816. Cluster summary at name node web has confusing report for space utilization. Contributed by Suresh Srinivas.

git-svn-id: https://svn.apache.org/repos/asf/hadoop/core/trunk@697232 13f79535-47bb-0310-9956-ffa450edef68
Hairong Kuang 17 yıl önce
ebeveyn
işleme
b268ae02b0

+ 7 - 0
CHANGES.txt

@@ -68,6 +68,13 @@ Trunk (unreleased changes)
     HADOOP-3722. Fixed Hadoop Streaming and Hadoop Pipes to use the Tool
     interface and GenericOptionsParser. (Enis Soztutar via acmurthy) 
 
+    HADOOP-2816. Cluster summary at name node web reports the space
+    utilization as:
+    Configured Capacity: capacity of all the data directories - Reserved space
+    Present Capacity: Space available for dfs,i.e. remaining+used space
+    DFS Used%: DFS used space/Present Capacity
+    (Suresh Srinivas via hairong)
+
   NEW FEATURES
 
     HADOOP-3341. Allow streaming jobs to specify the field separator for map

+ 12 - 0
src/hdfs/org/apache/hadoop/hdfs/protocol/DatanodeInfo.java

@@ -91,9 +91,21 @@ public class DatanodeInfo extends DatanodeID implements Node {
   /** The raw capacity. */
   public long getCapacity() { return capacity; }
   
+  /** The present capacity available for DFS. */
+  public long getPresentCapacity() { return dfsUsed + remaining; }
+  
   /** The used space by the data node. */
   public long getDfsUsed() { return dfsUsed; }
 
+  /** The used space by the data node as percentage of present capacity */
+  public float getDfsUsedPercent() { 
+    if (getPresentCapacity() <= 0) {
+      return 100;
+    }
+
+    return ((float)dfsUsed * 100.0f)/(float)getPresentCapacity(); 
+  }
+
   /** The raw free space. */
   public long getRemaining() { return remaining; }
 

+ 6 - 2
src/hdfs/org/apache/hadoop/hdfs/server/datanode/FSDataset.java

@@ -344,11 +344,15 @@ public class FSDataset implements FSConstants, FSDatasetInterface {
     }
     
     long getCapacity() throws IOException {
-      return usage.getCapacity();
+      if (reserved > usage.getCapacity()) {
+        return 0;
+      }
+
+      return usage.getCapacity()-reserved;
     }
       
     long getAvailable() throws IOException {
-      long remaining = getCapacity()-getDfsUsed()-reserved;
+      long remaining = getCapacity()-getDfsUsed();
       long available = usage.getAvailable();
       if (remaining>available) {
         remaining = available;

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

@@ -3240,6 +3240,15 @@ public class FSNamesystem implements FSConstants, FSNamesystemMBean {
     }
   }
 
+  /**
+   * Total raw bytes including non-dfs used space.
+   */
+  public long getPresentCapacity() {
+    synchronized (heartbeats) {
+      return this.capacityUsed + this.capacityRemaining;
+    }
+  }
+
   /**
    * Total used space by data nodes
    */
@@ -3248,6 +3257,18 @@ public class FSNamesystem implements FSConstants, FSNamesystemMBean {
       return this.capacityUsed;
     }
   }
+  /**
+   * Total used space by data nodes
+   */
+  public float getCapacityUsedPercent() {
+    synchronized(heartbeats){
+      if (getPresentCapacity() <= 0) {
+        return 100;
+      }
+
+      return ((float)getCapacityUsed() * 100.0f)/(float)getPresentCapacity();
+    }
+  }
   /**
    * Total non-used raw bytes.
    */

+ 8 - 2
src/hdfs/org/apache/hadoop/hdfs/server/protocol/DatanodeProtocol.java

@@ -35,9 +35,15 @@ import org.apache.hadoop.ipc.VersionedProtocol;
  **********************************************************************/
 public interface DatanodeProtocol extends VersionedProtocol {
   /**
-   * 17: Remove the request for block report.
+   * 18: In sendHeartbeat, the capacity parameter reported was sum of 
+   *     the filesystem disk space of all the data directories. This is 
+   *     changed to exclude the reserved capacity defined by 
+   *     dfs.datanode.du.reserved. 
+   *
+   *     The new capacity reported is sum of the filesystem disk space of 
+   *     all the data directories minus the reserved capacity.
    */
-  public static final long versionID = 17L;
+  public static final long versionID = 18L;
   
   // error code
   final static int NOTIFY = 0;

+ 8 - 0
src/test/org/apache/hadoop/hdfs/MiniDFSCluster.java

@@ -824,4 +824,12 @@ public class MiniDFSCluster {
     }
     return list;
   }
+
+  /**
+   * Access to the data directory used for Datanodes
+   * @throws IOException 
+   */
+  public String getDataDirectory() {
+    return data_dir.getAbsolutePath();
+  }
 }

+ 128 - 0
src/test/org/apache/hadoop/hdfs/server/namenode/TestNamenodeCapacityReport.java

@@ -0,0 +1,128 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.hadoop.hdfs.server.namenode;
+
+
+import java.io.File;
+import java.util.ArrayList;
+
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.fs.DF;
+import org.apache.hadoop.hdfs.MiniDFSCluster;
+import org.apache.hadoop.hdfs.server.namenode.DatanodeDescriptor;
+import org.apache.hadoop.hdfs.server.namenode.FSNamesystem;
+import junit.framework.TestCase;
+import org.apache.commons.logging.Log;
+import org.apache.commons.logging.LogFactory;
+
+
+
+/**
+ * This tests InterDataNodeProtocol for block handling. 
+ */
+public class TestNamenodeCapacityReport extends TestCase {
+  private static final Log LOG = LogFactory.getLog(TestNamenodeCapacityReport.class);
+
+  /**
+   * The following test first creates a file.
+   * It verifies the block information from a datanode.
+   * Then, it updates the block with new information and verifies again. 
+   */
+  public void testVolumeSize() throws Exception {
+    Configuration conf = new Configuration();
+    MiniDFSCluster cluster = null;
+
+    // Set aside fifth of the total capacity as reserved
+    long reserved = 10000;
+    conf.setLong("dfs.datanode.du.reserved", reserved);
+    
+    try {
+      cluster = new MiniDFSCluster(conf, 1, true, null);
+      cluster.waitActive();
+      
+      FSNamesystem namesystem = cluster.getNameNode().namesystem;
+      
+      // Ensure the data reported for each data node is right
+      ArrayList<DatanodeDescriptor> live = new ArrayList<DatanodeDescriptor>();
+      ArrayList<DatanodeDescriptor> dead = new ArrayList<DatanodeDescriptor>();
+      namesystem.DFSNodesStatus(live, dead);
+      
+      assertTrue(live.size() == 1);
+      
+      long used, remaining, totalCapacity, presentCapacity;
+      float percentUsed;
+      
+      for (final DatanodeDescriptor datanode : live) {
+        used = datanode.getDfsUsed();
+        remaining = datanode.getRemaining();
+        totalCapacity = datanode.getCapacity();
+        presentCapacity = datanode.getPresentCapacity();
+        percentUsed = datanode.getDfsUsedPercent();
+        
+        LOG.info("Datanode totalCapacity " + totalCapacity
+            + " presentCapacity " + presentCapacity + " used " + used
+            + " remaining " + remaining + " perenceUsed " + percentUsed);
+        
+        assertTrue(presentCapacity == (used + remaining));
+        assertTrue(percentUsed == ((100.0f * (float)used)/(float)presentCapacity));
+      }   
+      
+      DF df = new DF(new File(cluster.getDataDirectory()), conf);
+     
+      //
+      // Currently two data directories are created by the data node
+      // in the MiniDFSCluster. This results in each data directory having
+      // capacity equals to the disk capacity of the data directory.
+      // Hence the capacity reported by the data node is twice the disk space
+      // the disk capacity
+      //
+      // So multiply the disk capacity and reserved space by two 
+      // for accommodating it
+      //
+      int numOfDataDirs = 2;
+      
+      long diskCapacity = numOfDataDirs * df.getCapacity();
+      reserved *= numOfDataDirs;
+      
+      totalCapacity = namesystem.getCapacityTotal();
+      presentCapacity = namesystem.getPresentCapacity();
+      used = namesystem.getCapacityUsed();
+      remaining = namesystem.getCapacityRemaining();
+      percentUsed = namesystem.getCapacityUsedPercent();
+      
+      LOG.info("Data node directory " + cluster.getDataDirectory());
+           
+      LOG.info("Name node diskCapacity " + diskCapacity + " totalCapacity "
+          + totalCapacity + " reserved " + reserved + " presentCapacity "
+          + presentCapacity + " used " + used + " remaining " + remaining
+          + " percentUsed " + percentUsed);
+      
+      // Ensure new total capacity reported excludes the reserved space
+      assertTrue(totalCapacity == diskCapacity - reserved);
+      
+      // Ensure present capacity is sum of used and remaining
+      assertTrue(presentCapacity == (used + remaining));
+      
+      // Ensure percent used is calculated based on used and present capacity
+      assertTrue(percentUsed == ((float)used * 100.0f)/(float)presentCapacity);
+    }
+    finally {
+      if (cluster != null) {cluster.shutdown();}
+    }
+  }
+}

+ 30 - 26
src/webapps/hdfs/dfshealth.jsp

@@ -85,13 +85,9 @@
         return;
     
     long c = d.getCapacity();
+    long pc = d.getPresentCapacity();
     long u = d.getDfsUsed();
-    
-    String percentUsed;
-    if (c > 0) 
-      percentUsed = FsShell.limitDecimalTo2(((1.0 * u)/c)*100);
-    else
-      percentUsed = "100";
+    String percentUsed = FsShell.limitDecimalTo2(d.getDfsUsedPercent());    
     
     String adminState = (d.isDecommissioned() ? "Decommissioned" :
                          (d.isDecommissionInProgress() ? "Decommission In Progress":
@@ -105,6 +101,8 @@
               adminState +
 	      "<td class=\"size\">" +
               FsShell.limitDecimalTo2(c*1.0/diskBytes) +
+	      "<td align=\"right\" class=\"pcapacity\">" +
+              FsShell.limitDecimalTo2(pc*1.0/diskBytes) +      
 	      "<td class=\"pcused\">" + percentUsed +"<td class=\"pcused\">" +
 	      ServletUtil.percentageGraph( (int)Double.parseDouble(percentUsed) , 100) +
 	      "<td class=\"size\">" +
@@ -149,16 +147,23 @@
         
     counterReset();
     
+    long total = fsn.getCapacityTotal();
+    long present = fsn.getPresentCapacity();
+    long remaining = fsn.getCapacityRemaining();
+    long used = fsn.getCapacityUsed();
+    float percentUsed = fsn.getCapacityUsedPercent();
+
     out.print( "<div id=\"dfstable\"> <table>\n" +
-	       rowTxt() + colTxt() + "Capacity" + colTxt() + ":" + colTxt() +
-	       FsShell.byteDesc( fsn.getCapacityTotal() ) +
+	       rowTxt() + colTxt() + "Configured Capacity" + colTxt() + ":" + colTxt() +
+	       FsShell.byteDesc( total ) +
+	       rowTxt() + colTxt() + "Present Capacity" + colTxt() + ":" + colTxt() +
+	       FsShell.byteDesc( present ) +
 	       rowTxt() + colTxt() + "DFS Remaining" + colTxt() + ":" + colTxt() +
-	       FsShell.byteDesc( fsn.getCapacityRemaining() ) +
+	       FsShell.byteDesc( remaining ) +
 	       rowTxt() + colTxt() + "DFS Used" + colTxt() + ":" + colTxt() +
-	       FsShell.byteDesc( fsn.getCapacityUsed() ) +
+	       FsShell.byteDesc( used ) +
 	       rowTxt() + colTxt() + "DFS Used%" + colTxt() + ":" + colTxt() +
-	       FsShell.limitDecimalTo2((fsn.getCapacityUsed())*100.0/
-				       (fsn.getCapacityTotal() + 1e-10)) + " %" +
+	       FsShell.limitDecimalTo2(percentUsed) + " %" +
 	       rowTxt() + colTxt() +
                "<a href=\"#LiveNodes\">Live Nodes</a> " +
                colTxt() + ":" + colTxt() + live.size() +
@@ -168,7 +173,7 @@
                "</table></div><br><hr>\n" );
     
     if (live.isEmpty() && dead.isEmpty()) {
-	out.print("There are no datanodes in the cluster");
+        out.print("There are no datanodes in the cluster");
     }
     else {
         
@@ -190,10 +195,12 @@
                        ("name") + "> Node <th " +
                        NodeHeaderStr("lastcontact") + "> Last Contact <th " +
                        NodeHeaderStr("adminstate") + "> Admin State <th " +
-                       NodeHeaderStr("size") + "> Size (" + diskByteStr +
-                       ") <th " + NodeHeaderStr("pcused") +
-                       "> Used (%) <th " + NodeHeaderStr("pcused") +
-                       "> Used (%) <th " +
+                       NodeHeaderStr("size") + "> Configured capacity (" + 
+                       diskByteStr + ") <th " + 
+                       NodeHeaderStr("pcapacity") + "> Present capacity (" + 
+                       diskByteStr + ") <th " + 
+                       NodeHeaderStr("pcused") + "> Used (%) <th " + 
+                       NodeHeaderStr("pcused") + "> Used (%) <th " +
                        NodeHeaderStr("remaining") + "> Remaining (" + 
                        diskByteStr + ") <th " +
                        NodeHeaderStr("blocks") + "> Blocks\n" );
@@ -202,9 +209,9 @@
 		generateNodeData( out, live.get(i), port_suffix, true );
 	    }
 	}
-        out.print("</table>\n");
-
-        counterReset();
+    out.print("</table>\n");
+    
+    counterReset();
 	
 	out.print("<br> <a name=\"DeadNodes\" id=\"title\"> " +
                   " Dead Datanodes : " +dead.size() + "</a><br><br>\n");
@@ -239,19 +246,16 @@
 <tr> <td id="col1"> Upgrades: <td> <%= jspHelper.getUpgradeStatusText()%>
 </table></div><br>				      
 
-<b><a href="/nn_browsedfscontent.jsp">Browse the filesystem</a></b>
+<b><a href="/nn_browsedfscontent.jsp">Browse the filesystem</a></b><br>
+<b><a href="/logs/">Namenode Logs</a></b>
+
 <hr>
 <h3>Cluster Summary</h3>
 <b> <%= jspHelper.getSafeModeText()%> </b>
 <b> <%= jspHelper.getInodeLimitText()%> </b>
-
 <% 
     generateDFSHealthReport(out, request); 
 %>
-<hr>
-
-<h3>Local logs</h3>
-<a href="/logs/">Log</a> directory
 
 <%
 out.println(ServletUtil.htmlFooter());