You are viewing a plain text version of this content. The canonical link for it is here.
Posted to hdfs-commits@hadoop.apache.org by cm...@apache.org on 2014/08/20 01:50:25 UTC

svn commit: r1619012 [25/35] - in /hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project: hadoop-hdfs-httpfs/ hadoop-hdfs-httpfs/src/main/java/org/apache/hadoop/fs/http/client/ hadoop-hdfs-httpfs/src/main/java/org/apache/hadoop/fs/http/server/ hadoop...

Modified: hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/native/libhdfs/test_native_mini_dfs.c
URL: http://svn.apache.org/viewvc/hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/native/libhdfs/test_native_mini_dfs.c?rev=1619012&r1=1619011&r2=1619012&view=diff
==============================================================================
--- hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/native/libhdfs/test_native_mini_dfs.c (original)
+++ hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/native/libhdfs/test_native_mini_dfs.c Tue Aug 19 23:49:39 2014
@@ -22,7 +22,7 @@
 #include <errno.h>
 
 static struct NativeMiniDfsConf conf = {
-    .doFormat = 1,
+    1, /* doFormat */
 };
 
 /**

Modified: hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/proto/ClientNamenodeProtocol.proto
URL: http://svn.apache.org/viewvc/hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/proto/ClientNamenodeProtocol.proto?rev=1619012&r1=1619011&r2=1619012&view=diff
==============================================================================
--- hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/proto/ClientNamenodeProtocol.proto (original)
+++ hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/proto/ClientNamenodeProtocol.proto Tue Aug 19 23:49:39 2014
@@ -31,6 +31,7 @@ package hadoop.hdfs;
 import "Security.proto";
 import "hdfs.proto";
 import "acl.proto";
+import "xattr.proto";
 
 /**
  * The ClientNamenodeProtocol Service defines the interface between a client 
@@ -117,6 +118,7 @@ message AbandonBlockRequestProto {
   required ExtendedBlockProto b = 1;
   required string src = 2;
   required string holder = 3;
+  optional uint64 fileId = 4 [default = 0];  // default to GRANDFATHER_INODE_ID
 }
 
 message AbandonBlockResponseProto { // void response
@@ -143,6 +145,7 @@ message GetAdditionalDatanodeRequestProt
   required uint32 numAdditionalNodes = 5;
   required string clientName = 6;
   repeated string existingStorageUuids = 7;
+  optional uint64 fileId = 8 [default = 0];  // default to GRANDFATHER_INODE_ID
 }
 
 message GetAdditionalDatanodeResponseProto {
@@ -267,6 +270,7 @@ enum DatanodeReportTypeProto {  // type 
   ALL = 1;
   LIVE = 2;
   DEAD = 3;
+  DECOMMISSIONING = 4;
 }
 
 message GetDatanodeReportRequestProto {
@@ -277,6 +281,19 @@ message GetDatanodeReportResponseProto {
   repeated DatanodeInfoProto di = 1;
 }
 
+message GetDatanodeStorageReportRequestProto {
+  required DatanodeReportTypeProto type = 1;
+}
+
+message DatanodeStorageReportProto {
+  required DatanodeInfoProto datanodeInfo = 1;
+  repeated StorageReportProto storageReports = 2;
+}
+
+message GetDatanodeStorageReportResponseProto {
+  repeated DatanodeStorageReportProto datanodeStorageReports = 1;
+}
+
 message GetPreferredBlockSizeRequestProto {
   required string filename = 1;
 }
@@ -531,6 +548,7 @@ message FsyncRequestProto {
   required string src = 1;
   required string client = 2;
   optional sint64 lastBlockLength = 3 [default = -1];
+  optional uint64 fileId = 4 [default = 0];  // default to GRANDFATHER_INODE_ID
 }
 
 message FsyncResponseProto { // void response
@@ -636,6 +654,14 @@ message DeleteSnapshotRequestProto {
 message DeleteSnapshotResponseProto { // void response
 }
 
+message CheckAccessRequestProto {
+  required string path = 1;
+  required AclEntryProto.FsActionProto mode = 2;
+}
+
+message CheckAccessResponseProto { // void response
+}
+
 service ClientNamenodeProtocol {
   rpc getBlockLocations(GetBlockLocationsRequestProto)
       returns(GetBlockLocationsResponseProto);
@@ -667,6 +693,8 @@ service ClientNamenodeProtocol {
   rpc getFsStats(GetFsStatusRequestProto) returns(GetFsStatsResponseProto);
   rpc getDatanodeReport(GetDatanodeReportRequestProto)
       returns(GetDatanodeReportResponseProto);
+  rpc getDatanodeStorageReport(GetDatanodeStorageReportRequestProto)
+      returns(GetDatanodeStorageReportResponseProto);
   rpc getPreferredBlockSize(GetPreferredBlockSizeRequestProto)
       returns(GetPreferredBlockSizeResponseProto);
   rpc setSafeMode(SetSafeModeRequestProto)
@@ -755,4 +783,14 @@ service ClientNamenodeProtocol {
       returns(SetAclResponseProto);
   rpc getAclStatus(GetAclStatusRequestProto)
       returns(GetAclStatusResponseProto);
+  rpc setXAttr(SetXAttrRequestProto)
+      returns(SetXAttrResponseProto);
+  rpc getXAttrs(GetXAttrsRequestProto)
+      returns(GetXAttrsResponseProto);
+  rpc listXAttrs(ListXAttrsRequestProto)
+      returns(ListXAttrsResponseProto);
+  rpc removeXAttr(RemoveXAttrRequestProto)
+      returns(RemoveXAttrResponseProto);
+  rpc checkAccess(CheckAccessRequestProto)
+      returns(CheckAccessResponseProto);
 }

Modified: hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/proto/DatanodeProtocol.proto
URL: http://svn.apache.org/viewvc/hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/proto/DatanodeProtocol.proto?rev=1619012&r1=1619011&r2=1619012&view=diff
==============================================================================
--- hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/proto/DatanodeProtocol.proto (original)
+++ hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/proto/DatanodeProtocol.proto Tue Aug 19 23:49:39 2014
@@ -45,20 +45,6 @@ message DatanodeRegistrationProto {
 }
 
 /**
- * Represents a storage available on the datanode
- */
-message DatanodeStorageProto {
-  enum StorageState {
-    NORMAL = 0;
-    READ_ONLY_SHARED = 1;
-  }
-
-  required string storageUuid = 1;
-  optional StorageState state = 2 [default = NORMAL];
-  optional StorageTypeProto storageType = 3 [default = DISK];
-}
-
-/**
  * Commands sent from namenode to the datanodes
  */
 message DatanodeCommandProto {
@@ -113,6 +99,7 @@ message BlockCommandProto {
   repeated BlockProto blocks = 3;
   repeated DatanodeInfosProto targets = 4;
   repeated StorageUuidsProto targetStorageUuids = 5;
+  repeated StorageTypesProto targetStorageTypes = 6;
 }
 
 /**
@@ -195,16 +182,6 @@ message HeartbeatRequestProto {
   optional uint64 cacheUsed = 7 [default = 0 ];
 }
 
-message StorageReportProto {
-  required string storageUuid = 1 [ deprecated = true ];
-  optional bool failed = 2 [ default = false ];
-  optional uint64 capacity = 3 [ default = 0 ];
-  optional uint64 dfsUsed = 4 [ default = 0 ];
-  optional uint64 remaining = 5 [ default = 0 ];
-  optional uint64 blockPoolUsed = 6 [ default = 0 ];
-  optional DatanodeStorageProto storage = 7; // supersedes StorageUuid
-}
-
 /**
  * state - State the NN is in when returning response to the DN
  * txid - Highest transaction ID this NN has seen

Modified: hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/proto/datatransfer.proto
URL: http://svn.apache.org/viewvc/hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/proto/datatransfer.proto?rev=1619012&r1=1619011&r2=1619012&view=diff
==============================================================================
--- hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/proto/datatransfer.proto (original)
+++ hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/proto/datatransfer.proto Tue Aug 19 23:49:39 2014
@@ -107,17 +107,21 @@ message OpWriteBlockProto {
    */
   required ChecksumProto requestedChecksum = 9;
   optional CachingStrategyProto cachingStrategy = 10;
+  optional StorageTypeProto storageType = 11 [default = DISK];
+  repeated StorageTypeProto targetStorageTypes = 12;
 }
   
 message OpTransferBlockProto {
   required ClientOperationHeaderProto header = 1;
   repeated DatanodeInfoProto targets = 2;
+  repeated StorageTypeProto targetStorageTypes = 3;
 }
 
 message OpReplaceBlockProto {
   required BaseHeaderProto header = 1;
   required string delHint = 2;
   required DatanodeInfoProto source = 3;
+  optional StorageTypeProto storageType = 4 [default = DISK];
 }
 
 message OpCopyBlockProto {
@@ -203,6 +207,7 @@ enum Status {
   OOB_RESERVED1 = 9;          // Reserved
   OOB_RESERVED2 = 10;         // Reserved
   OOB_RESERVED3 = 11;         // Reserved
+  IN_PROGRESS = 12;
 }
 
 message PipelineAckProto {

Modified: hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/proto/fsimage.proto
URL: http://svn.apache.org/viewvc/hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/proto/fsimage.proto?rev=1619012&r1=1619011&r2=1619012&view=diff
==============================================================================
--- hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/proto/fsimage.proto (original)
+++ hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/proto/fsimage.proto Tue Aug 19 23:49:39 2014
@@ -23,6 +23,7 @@ package hadoop.hdfs.fsimage;
 
 import "hdfs.proto";
 import "acl.proto";
+import "xattr.proto";
 
 /**
  * This file defines the on-disk layout of the file system image. The
@@ -106,7 +107,23 @@ message INodeSection {
      */
     repeated fixed32 entries = 2 [packed = true];
   }
-
+  
+  message XAttrCompactProto {
+    /**
+     * 
+     * [0:2) -- the namespace of XAttr (XAttrNamespaceProto)
+     * [2:26) -- the name of the entry, which is an ID that points to a
+     * string in the StringTableSection. 
+     * [26:32) -- reserved for future uses.
+     */
+    required fixed32 name = 1;
+    optional bytes value = 2;
+  }
+  
+  message XAttrFeatureProto {
+    repeated XAttrCompactProto xAttrs = 1;
+  }
+  
   message INodeFile {
     optional uint32 replication = 1;
     optional uint64 modificationTime = 2;
@@ -116,6 +133,7 @@ message INodeSection {
     repeated BlockProto blocks = 6;
     optional FileUnderConstructionFeature fileUC = 7;
     optional AclFeatureProto acl = 8;
+    optional XAttrFeatureProto xAttrs = 9;
   }
 
   message INodeDirectory {
@@ -126,6 +144,7 @@ message INodeSection {
     optional uint64 dsQuota = 3;
     optional fixed64 permission = 4;
     optional AclFeatureProto acl = 5;
+    optional XAttrFeatureProto xAttrs = 6;
   }
 
   message INodeSymlink {

Modified: hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/proto/hdfs.proto
URL: http://svn.apache.org/viewvc/hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/proto/hdfs.proto?rev=1619012&r1=1619011&r2=1619012&view=diff
==============================================================================
--- hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/proto/hdfs.proto (original)
+++ hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/proto/hdfs.proto Tue Aug 19 23:49:39 2014
@@ -100,6 +100,30 @@ message DatanodeInfoProto {
 }
 
 /**
+ * Represents a storage available on the datanode
+ */
+message DatanodeStorageProto {
+  enum StorageState {
+    NORMAL = 0;
+    READ_ONLY_SHARED = 1;
+  }
+
+  required string storageUuid = 1;
+  optional StorageState state = 2 [default = NORMAL];
+  optional StorageTypeProto storageType = 3 [default = DISK];
+}
+
+message StorageReportProto {
+  required string storageUuid = 1 [ deprecated = true ];
+  optional bool failed = 2 [ default = false ];
+  optional uint64 capacity = 3 [ default = 0 ];
+  optional uint64 dfsUsed = 4 [ default = 0 ];
+  optional uint64 remaining = 5 [ default = 0 ];
+  optional uint64 blockPoolUsed = 6 [ default = 0 ];
+  optional DatanodeStorageProto storage = 7; // supersedes StorageUuid
+}
+
+/**
  * Summary of a file or directory
  */
 message ContentSummaryProto {
@@ -137,6 +161,13 @@ enum StorageTypeProto {
 }
 
 /**
+ * A list of storage types. 
+ */
+message StorageTypesProto {
+  repeated StorageTypeProto storageTypes = 1;
+}
+
+/**
  * A list of storage IDs. 
  */
 message StorageUuidsProto {
@@ -275,6 +306,7 @@ message SnapshottableDirectoryListingPro
 message SnapshotDiffReportEntryProto {
   required bytes fullpath = 1;
   required string modificationLabel = 2;
+  optional bytes targetPath = 3;
 }
 
 /**
@@ -372,6 +404,7 @@ message BlockWithLocationsProto {
   required BlockProto block = 1;   // Block
   repeated string datanodeUuids = 2; // Datanodes with replicas of the block
   repeated string storageUuids = 3;  // Storages with replicas of the block
+  repeated StorageTypeProto storageTypes = 4;
 }
 
 /**

Modified: hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/resources/META-INF/services/org.apache.hadoop.security.token.TokenIdentifier
URL: http://svn.apache.org/viewvc/hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/resources/META-INF/services/org.apache.hadoop.security.token.TokenIdentifier?rev=1619012&r1=1619011&r2=1619012&view=diff
==============================================================================
--- hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/resources/META-INF/services/org.apache.hadoop.security.token.TokenIdentifier (original)
+++ hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/resources/META-INF/services/org.apache.hadoop.security.token.TokenIdentifier Tue Aug 19 23:49:39 2014
@@ -13,3 +13,5 @@
 #
 org.apache.hadoop.hdfs.security.token.block.BlockTokenIdentifier
 org.apache.hadoop.hdfs.security.token.delegation.DelegationTokenIdentifier
+org.apache.hadoop.hdfs.security.token.delegation.DelegationTokenIdentifier$WebHdfsDelegationTokenIdentifier
+org.apache.hadoop.hdfs.security.token.delegation.DelegationTokenIdentifier$SWebHdfsDelegationTokenIdentifier

Modified: hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/resources/hdfs-default.xml
URL: http://svn.apache.org/viewvc/hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/resources/hdfs-default.xml?rev=1619012&r1=1619011&r2=1619012&view=diff
==============================================================================
--- hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/resources/hdfs-default.xml (original)
+++ hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/resources/hdfs-default.xml Tue Aug 19 23:49:39 2014
@@ -55,11 +55,11 @@
   <name>dfs.namenode.rpc-bind-host</name>
   <value></value>
   <description>
-    The actual address the server will bind to. If this optional address is
-    set, the RPC server will bind to this address and the port specified in
-    dfs.namenode.rpc-address for the RPC server. It can also be specified
-    per name node or name service for HA/Federation. This is most useful for
-    making name node listen to all interfaces by setting to 0.0.0.0.
+    The actual address the RPC server will bind to. If this optional address is
+    set, it overrides only the hostname portion of dfs.namenode.rpc-address.
+    It can also be specified per name node or name service for HA/Federation.
+    This is useful for making the name node listen on all interfaces by
+    setting it to 0.0.0.0.
   </description>
 </property>
 
@@ -80,11 +80,11 @@
   <name>dfs.namenode.servicerpc-bind-host</name>
   <value></value>
   <description>
-    The actual address the server will bind to. If this optional address is
-    set, the service RPC server will bind to this address and the port 
-    specified in dfs.namenode.servicerpc-address. It can also be specified
-    per name node or name service for HA/Federation. This is most useful for
-    making name node listen to all interfaces by setting to 0.0.0.0.
+    The actual address the service RPC server will bind to. If this optional address is
+    set, it overrides only the hostname portion of dfs.namenode.servicerpc-address.
+    It can also be specified per name node or name service for HA/Federation.
+    This is useful for making the name node listen on all interfaces by
+    setting it to 0.0.0.0.
   </description>
 </property>
 
@@ -143,6 +143,18 @@
 </property>
 
 <property>
+  <name>dfs.namenode.http-bind-host</name>
+  <value></value>
+  <description>
+    The actual adress the HTTP server will bind to. If this optional address
+    is set, it overrides only the hostname portion of dfs.namenode.http-address.
+    It can also be specified per name node or name service for HA/Federation.
+    This is useful for making the name node HTTP server listen on all
+    interfaces by setting it to 0.0.0.0.
+  </description>
+</property>
+
+<property>
   <name>dfs.https.enable</name>
   <value>false</value>
   <description>
@@ -207,6 +219,18 @@
   <description>The namenode secure http server address and port.</description>
 </property>
 
+<property>
+  <name>dfs.namenode.https-bind-host</name>
+  <value></value>
+  <description>
+    The actual adress the HTTPS server will bind to. If this optional address
+    is set, it overrides only the hostname portion of dfs.namenode.https-address.
+    It can also be specified per name node or name service for HA/Federation.
+    This is useful for making the name node HTTPS server listen on all
+    interfaces by setting it to 0.0.0.0.
+  </description>
+</property>
+
  <property>
   <name>dfs.datanode.dns.interface</name>
   <value>default</value>
@@ -1272,8 +1296,56 @@
     non-zero integer.
   </description>
 </property>
+
+<property>
+  <name>nfs.server.port</name>
+  <value>2049</value>
+  <description>
+      Specify the port number used by Hadoop NFS.
+  </description>
+</property>
+
+<property>
+  <name>nfs.mountd.port</name>
+  <value>4242</value>
+  <description>
+      Specify the port number used by Hadoop mount daemon.
+  </description>
+</property>
+
+<property>    
+  <name>nfs.dump.dir</name>
+  <value>/tmp/.hdfs-nfs</value>
+  <description>
+    This directory is used to temporarily save out-of-order writes before
+    writing to HDFS. For each file, the out-of-order writes are dumped after
+    they are accumulated to exceed certain threshold (e.g., 1MB) in memory. 
+    One needs to make sure the directory has enough space.
+  </description>
+</property>
+
+<property>
+  <name>nfs.rtmax</name>
+  <value>1048576</value>
+  <description>This is the maximum size in bytes of a READ request
+    supported by the NFS gateway. If you change this, make sure you
+    also update the nfs mount's rsize(add rsize= # of bytes to the 
+    mount directive).
+  </description>
+</property>
+
+<property>
+  <name>nfs.wtmax</name>
+  <value>1048576</value>
+  <description>This is the maximum size in bytes of a WRITE request
+    supported by the NFS gateway. If you change this, make sure you
+    also update the nfs mount's wsize(add wsize= # of bytes to the 
+    mount directive).
+  </description>
+</property>
+
 <property>
-  <name>dfs.nfs.keytab.file</name>
+  <name>nfs.keytab.file</name>
   <value></value>
   <description>
     *Note*: Advanced property. Change with caution.
@@ -1283,7 +1355,7 @@
 </property>
 
 <property>
-  <name>dfs.nfs.kerberos.principal</name>
+  <name>nfs.kerberos.principal</name>
   <value></value>
   <description>
     *Note*: Advanced property. Change with caution.
@@ -1294,6 +1366,17 @@
 </property>
 
 <property>
+  <name>nfs.allow.insecure.ports</name>
+  <value>true</value>
+  <description>
+    When set to false, client connections originating from unprivileged ports
+    (those above 1023) will be rejected. This is to ensure that clients
+    connecting to this NFS Gateway must have had root privilege on the machine
+    where they're connecting from.
+  </description>
+</property>
+
+<property>
   <name>dfs.webhdfs.enabled</name>
   <value>true</value>
   <description>
@@ -1369,6 +1452,37 @@
 </property>
 
 <property>
+  <name>dfs.data.transfer.protection</name>
+  <value></value>
+  <description>
+    A comma-separated list of SASL protection values used for secured
+    connections to the DataNode when reading or writing block data.  Possible
+    values are authentication, integrity and privacy.  authentication means
+    authentication only and no integrity or privacy; integrity implies
+    authentication and integrity are enabled; and privacy implies all of
+    authentication, integrity and privacy are enabled.  If
+    dfs.encrypt.data.transfer is set to true, then it supersedes the setting for
+    dfs.data.transfer.protection and enforces that all connections must use a
+    specialized encrypted SASL handshake.  This property is ignored for
+    connections to a DataNode listening on a privileged port.  In this case, it
+    is assumed that the use of a privileged port establishes sufficient trust.
+  </description>
+</property>
+
+<property>
+  <name>dfs.data.transfer.saslproperties.resolver.class</name>
+  <value></value>
+  <description>
+    SaslPropertiesResolver used to resolve the QOP used for a connection to the
+    DataNode when reading or writing block data.  If not specified, the full set
+    of values specified in dfs.data.transfer.protection is used while
+    determining the QOP used for the connection. If a class is specified, then
+    the QOP values returned by the class will be used while determining the QOP
+    used for the connection.
+  </description>
+</property>
+
+<property>
   <name>dfs.datanode.hdfs-blocks-metadata.enabled</name>
   <value>false</value>
   <description>
@@ -1871,4 +1985,79 @@
   </description>
 </property>
 
-</configuration>
\ No newline at end of file
+<property>
+  <name>dfs.namenode.xattrs.enabled</name>
+  <value>true</value>
+  <description>
+    Whether support for extended attributes is enabled on the NameNode.
+  </description>
+</property>
+
+<property>
+  <name>dfs.namenode.fs-limits.max-xattrs-per-inode</name>
+  <value>32</value>
+  <description>
+    Maximum number of extended attributes per inode.
+  </description>
+</property>
+
+<property>
+  <name>dfs.namenode.fs-limits.max-xattr-size</name>
+  <value>16384</value>
+  <description>
+    The maximum combined size of the name and value of an extended attribute in bytes.
+  </description>
+</property>
+
+<property>
+  <name>dfs.client.slow.io.warning.threshold.ms</name>
+  <value>30000</value>
+  <description>The threshold in milliseconds at which we will log a slow
+    io warning in a dfsclient. By default, this parameter is set to 30000
+    milliseconds (30 seconds).
+  </description>
+</property>
+
+<property>
+  <name>dfs.datanode.slow.io.warning.threshold.ms</name>
+  <value>300</value>
+  <description>The threshold in milliseconds at which we will log a slow
+    io warning in a datanode. By default, this parameter is set to 300
+    milliseconds.
+  </description>
+</property>
+
+<property>
+  <name>dfs.namenode.startup.delay.block.deletion.sec</name>
+  <value>0</value>
+  <description>The delay in seconds at which we will pause the blocks deletion
+    after Namenode startup. By default it's disabled.
+    In the case a directory has large number of directories and files are
+    deleted, suggested delay is one hour to give the administrator enough time
+    to notice large number of pending deletion blocks and take corrective
+    action.
+  </description>
+</property>
+
+<property>
+  <name>dfs.namenode.randomize-block-locations-per-block</name>
+  <value>false</value>
+  <description>When fetching replica locations of a block, the replicas
+   are sorted based on network distance. This configuration parameter
+   determines whether the replicas at the same network distance are randomly
+   shuffled. By default, this is false, such that repeated requests for a block's
+   replicas always result in the same order. This potentially improves page cache
+   behavior. However, for some network topologies, it is desirable to shuffle this
+   order for better load balancing.
+  </description>
+</property>
+
+<property>
+  <name>dfs.datanode.block.id.layout.upgrade.threads</name>
+  <value>12</value>
+  <description>The number of threads to use when creating hard links from
+    current to previous blocks during upgrade of a DataNode to block ID-based
+    block layout (see HDFS-6482 for details on the layout).</description>
+</property>
+
+</configuration>

Propchange: hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/datanode/
------------------------------------------------------------------------------
  Merged /hadoop/common/trunk/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/datanode:r1582150-1619000
  Merged /hadoop/common/branches/HDFS-2006/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/datanode:r1588992-1596568

Modified: hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/datanode/index.html
URL: http://svn.apache.org/viewvc/hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/datanode/index.html?rev=1619012&r1=1619011&r2=1619012&view=diff
==============================================================================
--- hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/datanode/index.html (original)
+++ hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/datanode/index.html Tue Aug 19 23:49:39 2014
@@ -1,3 +1,5 @@
+<!DOCTYPE html PUBLIC "-//W3C//DTD XHTML 1.0 Strict//EN"
+    "http://www.w3.org/TR/xhtml1/DTD/xhtml1-strict.dtd">
 <!--
    Licensed to the Apache Software Foundation (ASF) under one or more
    contributor license agreements.  See the NOTICE file distributed with
@@ -14,22 +16,47 @@
    See the License for the specific language governing permissions and
    limitations under the License.
 -->
-<meta HTTP-EQUIV="REFRESH" content="0;url=dataNodeHome.jsp"/>
-<html>
+<html xmlns="http://www.w3.org/1999/xhtml">
 <head>
-<title>Hadoop Administration</title>
+<meta http-equiv="X-UA-Compatible" content="IE=9" />
+<link rel="stylesheet" type="text/css" href="/static/bootstrap-3.0.2/css/bootstrap.min.css" />
+<link rel="stylesheet" type="text/css" href="/static/hadoop.css" />
+<title>DataNode Information</title>
 </head>
-
 <body>
 
-<h1>Hadoop Administration</h1>
-
-<ul>
-
-<li><a href="dataNodeHome.jsp">DataNode Home</a></li>
-
-</ul>
-
+<header class="navbar navbar-inverse bs-docs-nav" role="banner">
+<div class="container">
+  <div class="navbar-header">
+    <div class="navbar-brand">Hadoop</div>
+  </div>
+
+  <ul class="nav navbar-nav" id="ui-tabs">
+    <li><a>Overview</a></li>
+  </ul>
+</div>
+</header>
+
+<div class="container">
+
+<div class="tab-content">
+  <div class="tab-pane" id="tab-overview">
+    <div class="page-header"><h1>DataNode on <small><div id="authority" style="display: inline-block"></div></small></h1></div>
+  </div>
+</div>
+
+<div class="row">
+  <hr />
+  <div class="col-xs-2"><p>Hadoop, 2014.</p></div>
+</div>
+</div>
+
+<script type="text/javascript" src="/static/jquery-1.10.2.min.js">
+</script><script type="text/javascript" src="/static/bootstrap-3.0.2/js/bootstrap.min.js">
+</script>
+<script type="text/javascript">
+$('#authority').html(window.location.host);
+$('#tab-overview').addClass('active');
+</script>
 </body>
-
 </html>

Propchange: hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/hdfs/
------------------------------------------------------------------------------
  Merged /hadoop/common/trunk/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/hdfs:r1588388-1619000
  Merged /hadoop/common/branches/HDFS-2006/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/hdfs:r1588992-1596568

Modified: hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/hdfs/dfshealth.html
URL: http://svn.apache.org/viewvc/hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/hdfs/dfshealth.html?rev=1619012&r1=1619011&r2=1619012&view=diff
==============================================================================
--- hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/hdfs/dfshealth.html (original)
+++ hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/hdfs/dfshealth.html Tue Aug 19 23:49:39 2014
@@ -18,6 +18,7 @@
 -->
 <html xmlns="http://www.w3.org/1999/xhtml">
 <head>
+<meta http-equiv="X-UA-Compatible" content="IE=9" />
 <link rel="stylesheet" type="text/css" href="/static/bootstrap-3.0.2/css/bootstrap.min.css" />
 <link rel="stylesheet" type="text/css" href="/static/hadoop.css" />
 <title>Namenode information</title>
@@ -65,14 +66,13 @@
 <div class="row">
   <hr />
   <div class="col-xs-2"><p>Hadoop, 2014.</p></div>
-  <div class="col-xs-1 pull-right"><a style="color: #ddd" href="dfshealth.jsp">Legacy UI</a></div>
 </div>
 </div>
 
 <script type="text/x-dust-template" id="tmpl-dfshealth">
 
 {#nn}
-{@if cond="{DistinctVersionCount} > 1 || '{RollingUpgradeStatus}'.length"}
+{@if cond="{DistinctVersionCount} > 1 || '{RollingUpgradeStatus}'.length || !'{UpgradeFinalized}'"}
 <div class="alert alert-dismissable alert-info">
   <button type="button" class="close" data-dismiss="alert" aria-hidden="true">&times;</button>
 
@@ -92,6 +92,10 @@
     {key} ({value}) {@sep},{/sep}
     {/DistinctVersions}
   {/if}
+
+  {^UpgradeFinalized}
+     <p>Upgrade in progress. Not yet finalized.</p>
+  {/UpgradeFinalized}
 </div>
 {/if}
 
@@ -277,8 +281,8 @@
   {#DeadNodes}
   <tr class="danger">
     <td>{name} ({xferaddr})</td>
-    <td>{lastContact}</td>
-    <td>Dead{?decomissioned}, Decomissioned{/decomissioned}</td>
+    <td>{#helper_lastcontact_tostring value="{lastContact}"/}</td>
+    <td>Dead{?decommissioned}, Decommissioned{/decommissioned}</td>
     <td>-</td>
     <td>-</td>
     <td>-</td>
@@ -351,7 +355,7 @@
 </script><script type="text/javascript" src="/static/bootstrap-3.0.2/js/bootstrap.min.js">
 </script><script type="text/javascript" src="/static/dust-full-2.0.0.min.js">
 </script><script type="text/javascript" src="/static/dust-helpers-1.1.1.min.js">
-</script><script type="text/javascript" src="dfs-dust.js">
+</script><script type="text/javascript" src="/static/dfs-dust.js">
 </script><script type="text/javascript" src="dfshealth.js">
 </script>
 </body>

Modified: hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/hdfs/dfshealth.js
URL: http://svn.apache.org/viewvc/hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/hdfs/dfshealth.js?rev=1619012&r1=1619011&r2=1619012&view=diff
==============================================================================
--- hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/hdfs/dfshealth.js (original)
+++ hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/hdfs/dfshealth.js Tue Aug 19 23:49:39 2014
@@ -139,6 +139,14 @@
   }
 
   function load_datanode_info() {
+
+    var HELPERS = {
+      'helper_lastcontact_tostring' : function (chunk, ctx, bodies, params) {
+        var value = dust.helpers.tap(params.value, chunk, ctx);
+        return chunk.write('' + new Date(Date.now()-1000*Number(value)));
+      }
+    };
+
     function workaround(r) {
       function node_map_to_array(nodes) {
         var res = [];
@@ -160,7 +168,8 @@
       '/jmx?qry=Hadoop:service=NameNode,name=NameNodeInfo',
       guard_with_startup_progress(function (resp) {
         var data = workaround(resp.beans[0]);
-        dust.render('datanode-info', data, function(err, out) {
+        var base = dust.makeBase(HELPERS);
+        dust.render('datanode-info', base.push(data), function(err, out) {
           $('#tab-datanode').html(out);
           $('#ui-tabs a[href="#tab-datanode"]').tab('show');
         });

Modified: hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/hdfs/explorer.html
URL: http://svn.apache.org/viewvc/hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/hdfs/explorer.html?rev=1619012&r1=1619011&r2=1619012&view=diff
==============================================================================
--- hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/hdfs/explorer.html (original)
+++ hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/hdfs/explorer.html Tue Aug 19 23:49:39 2014
@@ -18,11 +18,35 @@
   -->
 <html xmlns="http://www.w3.org/1999/xhtml">
   <head>
+    <meta http-equiv="X-UA-Compatible" content="IE=9" />
     <link rel="stylesheet" type="text/css" href="/static/bootstrap-3.0.2/css/bootstrap.min.css" />
     <link rel="stylesheet" type="text/css" href="/static/hadoop.css" />
     <title>Browsing HDFS</title>
   </head>
   <body>
+
+    <header class="navbar navbar-inverse bs-docs-nav" role="banner">
+    <div class="container">
+      <div class="navbar-header">
+        <div class="navbar-brand">Hadoop</div>
+      </div>
+
+      <ul class="nav navbar-nav" id="ui-tabs">
+        <li><a href="dfshealth.html#tab-overview">Overview</a></li>
+        <li><a href="dfshealth.html#tab-datanode">Datanodes</a></li>
+        <li><a href="dfshealth.html#tab-snapshot">Snapshot</a></li>
+        <li><a href="dfshealth.html#tab-startup-progress">Startup Progress</a></li>
+        <li class="dropdown">
+          <a href="#" class="dropdown-toggle" data-toggle="dropdown">Utilities <b class="caret"></b></a>
+          <ul class="dropdown-menu">
+            <li><a href="#">Browse the file system</a></li>
+            <li><a href="logs">Logs</a></li>
+          </ul>
+        </li>
+      </ul>
+    </div>
+    </header>
+
     <div class="modal" id="file-info" tabindex="-1" role="dialog" aria-hidden="true">
       <div class="modal-dialog">
 	<div class="modal-content">
@@ -72,6 +96,12 @@
       </div>
       <br />
       <div id="panel"></div>
+
+      <div class="row">
+        <hr />
+        <div class="col-xs-2"><p>Hadoop, 2014.</p></div>
+      </div>
+
     </div>
 
     <script type="text/x-dust-template" id="tmpl-explorer">
@@ -90,7 +120,7 @@
         <tbody>
           {#FileStatus}
           <tr>
-            <td>{type|helper_to_directory}{permission|helper_to_permission}</td>
+            <td>{type|helper_to_directory}{permission|helper_to_permission}{aclBit|helper_to_acl_bit}</td>
             <td>{owner}</td>
             <td>{group}</td>
             <td>{length|fmt_bytes}</td>
@@ -122,10 +152,8 @@
     </script><script type="text/javascript" src="/static/bootstrap-3.0.2/js/bootstrap.min.js">
     </script><script type="text/javascript" src="/static/dust-full-2.0.0.min.js">
     </script><script type="text/javascript" src="/static/dust-helpers-1.1.1.min.js">
-    </script><script type="text/javascript" src="dfs-dust.js">
+    </script><script type="text/javascript" src="/static/dfs-dust.js">
     </script><script type="text/javascript" src="explorer.js">
     </script>
-    <hr />
-    <p>Hadoop, 2014.</p>
   </body>
 </html>

Modified: hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/hdfs/explorer.js
URL: http://svn.apache.org/viewvc/hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/hdfs/explorer.js?rev=1619012&r1=1619011&r2=1619012&view=diff
==============================================================================
--- hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/hdfs/explorer.js (original)
+++ hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/hdfs/explorer.js Tue Aug 19 23:49:39 2014
@@ -45,7 +45,7 @@
     return function (jqxhr, text, err) {
       switch(jqxhr.status) {
         case 401:
-          var msg = '<p>Authentication failed when trying to open ' + url + ': Unauthrozied.</p>';
+          var msg = '<p>Authentication failed when trying to open ' + url + ': Unauthorized.</p>';
           break;
         case 403:
           if(jqxhr.responseJSON !== undefined && jqxhr.responseJSON.RemoteException !== undefined) {
@@ -58,7 +58,7 @@
           var msg = '<p>Path does not exist on HDFS or WebHDFS is disabled.  Please check your path or enable WebHDFS</p>';
           break;
         default:
-          var msg = '<p>Failed to retreive data from ' + url + ': ' + err + '</p>';
+          var msg = '<p>Failed to retrieve data from ' + url + ': ' + err + '</p>';
         }
       show_err_msg(msg);
     };
@@ -75,8 +75,7 @@
   }
 
   function get_response_err_msg(data) {
-    var msg = data.RemoteException !== undefined ? data.RemoteException.message : "";
-    return msg;
+    return data.RemoteException !== undefined ? data.RemoteException.message : "";
   }
 
   function view_file_details(path, abs_path) {
@@ -104,7 +103,7 @@
     }
 
     var url = '/webhdfs/v1' + abs_path + '?op=GET_BLOCK_LOCATIONS';
-    $.ajax({"url": url, "crossDomain": true}).done(function(data) {
+    $.get(url).done(function(data) {
       var d = get_response(data, "LocatedBlocks");
       if (d === null) {
         show_err_msg(get_response_err_msg(data));

Modified: hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/hdfs/index.html
URL: http://svn.apache.org/viewvc/hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/hdfs/index.html?rev=1619012&r1=1619011&r2=1619012&view=diff
==============================================================================
--- hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/hdfs/index.html (original)
+++ hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/hdfs/index.html Tue Aug 19 23:49:39 2014
@@ -18,18 +18,7 @@
     "http://www.w3.org/TR/xhtml1/DTD/xhtml1-strict.dtd">
 <html xmlns="http://www.w3.org/1999/xhtml">
 <head>
-<meta http-equiv="REFRESH" content="1;url=dfshealth.jsp" />
+<meta http-equiv="REFRESH" content="0;url=dfshealth.html" />
 <title>Hadoop Administration</title>
 </head>
-<body>
-<script type="text/javascript">
-//<![CDATA[
-window.location.href='dfshealth.html';
-//]]>
-</script>
-<h1>Hadoop Administration</h1>
-<ul>
-<li><a href="dfshealth.jsp">DFS Health/Status</a></li>
-</ul>
-</body>
 </html>

Modified: hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/journal/index.html
URL: http://svn.apache.org/viewvc/hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/journal/index.html?rev=1619012&r1=1619011&r2=1619012&view=diff
==============================================================================
--- hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/journal/index.html (original)
+++ hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/journal/index.html Tue Aug 19 23:49:39 2014
@@ -1,5 +1,5 @@
-<meta HTTP-EQUIV="REFRESH" content="0;url=journalstatus.jsp"/>
-<html>
+<!DOCTYPE html PUBLIC "-//W3C//DTD XHTML 1.0 Strict//EN"
+    "http://www.w3.org/TR/xhtml1/DTD/xhtml1-strict.dtd">
 <!--
    Licensed to the Apache Software Foundation (ASF) under one or more
    contributor license agreements.  See the NOTICE file distributed with
@@ -16,14 +16,47 @@
    See the License for the specific language governing permissions and
    limitations under the License.
 -->
-<head><title>Hadoop Administration</title></head>
-
+<html xmlns="http://www.w3.org/1999/xhtml">
+<head>
+<meta http-equiv="X-UA-Compatible" content="IE=9" />
+<link rel="stylesheet" type="text/css" href="/static/bootstrap-3.0.2/css/bootstrap.min.css" />
+<link rel="stylesheet" type="text/css" href="/static/hadoop.css" />
+<title>JournalNode Information</title>
+</head>
 <body>
-<h1>Hadoop Administration</h1>
 
-<ul> 
-  <li><a href="journalstatus.jsp">Status</a></li> 
-</ul>
+<header class="navbar navbar-inverse bs-docs-nav" role="banner">
+<div class="container">
+  <div class="navbar-header">
+    <div class="navbar-brand">Hadoop</div>
+  </div>
+
+  <ul class="nav navbar-nav" id="ui-tabs">
+    <li><a>Overview</a></li>
+  </ul>
+</div>
+</header>
+
+<div class="container">
+
+<div class="tab-content">
+  <div class="tab-pane" id="tab-overview">
+    <div class="page-header"><h1>JournalNode on <small><div id="authority" style="display: inline-block"></div></small></h1></div>
+  </div>
+</div>
+
+<div class="row">
+  <hr />
+  <div class="col-xs-2"><p>Hadoop, 2014.</p></div>
+</div>
+</div>
 
-</body> 
+<script type="text/javascript" src="/static/jquery-1.10.2.min.js">
+</script><script type="text/javascript" src="/static/bootstrap-3.0.2/js/bootstrap.min.js">
+</script>
+<script type="text/javascript">
+$('#authority').html(window.location.host);
+$('#tab-overview').addClass('active');
+</script>
+</body>
 </html>

Propchange: hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/secondary/
------------------------------------------------------------------------------
  Merged /hadoop/common/trunk/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/secondary:r1582150-1619000
  Merged /hadoop/common/branches/HDFS-2006/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/secondary:r1588992-1596568

Modified: hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/secondary/index.html
URL: http://svn.apache.org/viewvc/hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/secondary/index.html?rev=1619012&r1=1619011&r2=1619012&view=diff
==============================================================================
--- hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/secondary/index.html (original)
+++ hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/secondary/index.html Tue Aug 19 23:49:39 2014
@@ -1,5 +1,3 @@
-<meta HTTP-EQUIV="REFRESH" content="0;url=status.jsp"/>
-<html>
 <!--
    Licensed to the Apache Software Foundation (ASF) under one or more
    contributor license agreements.  See the NOTICE file distributed with
@@ -16,14 +14,11 @@
    See the License for the specific language governing permissions and
    limitations under the License.
 -->
-<head><title>Hadoop Administration</title></head>
-
-<body>
-<h1>Hadoop Administration</h1>
-
-<ul> 
-  <li><a href="status.jsp">Status</a></li> 
-</ul>
-
-</body> 
-</html>
+<!DOCTYPE html PUBLIC "-//W3C//DTD XHTML 1.0 Strict//EN"
+        "http://www.w3.org/TR/xhtml1/DTD/xhtml1-strict.dtd">
+<html xmlns="http://www.w3.org/1999/xhtml">
+<head>
+  <meta http-equiv="REFRESH" content="0;url=status.html" />
+  <title>Hadoop Administration</title>
+</head>
+</html>
\ No newline at end of file

Modified: hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/site/apt/CentralizedCacheManagement.apt.vm
URL: http://svn.apache.org/viewvc/hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/site/apt/CentralizedCacheManagement.apt.vm?rev=1619012&r1=1619011&r2=1619012&view=diff
==============================================================================
--- hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/site/apt/CentralizedCacheManagement.apt.vm (original)
+++ hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/site/apt/CentralizedCacheManagement.apt.vm Tue Aug 19 23:49:39 2014
@@ -18,8 +18,6 @@
 
 Centralized Cache Management in HDFS
 
-  \[ {{{./index.html}Go Back}} \]
-
 %{toc|section=1|fromDepth=2|toDepth=4}
 
 * {Overview}
@@ -272,7 +270,7 @@ Centralized Cache Management in HDFS
 ** {Native Libraries}
 
   In order to lock block files into memory, the DataNode relies on native JNI
-  code found in <<<libhadoop.so>>>. Be sure to
+  code found in <<<libhadoop.so>>> or <<<hadoop.dll>>> on Windows. Be sure to
   {{{../hadoop-common/NativeLibraries.html}enable JNI}} if you are using HDFS
   centralized cache management.
 
@@ -285,11 +283,11 @@ Centralized Cache Management in HDFS
   * dfs.datanode.max.locked.memory
 
     This determines the maximum amount of memory a DataNode will use for caching.
-    The "locked-in-memory size" ulimit (<<<ulimit -l>>>) of the DataNode user
-    also needs to be increased to match this parameter (see below section on
-    {{OS Limits}}). When setting this value, please remember that you will need
-    space in memory for other things as well, such as the DataNode and
-    application JVM heaps and the operating system page cache.
+    On Unix-like systems, the "locked-in-memory size" ulimit (<<<ulimit -l>>>) of
+    the DataNode user also needs to be increased to match this parameter (see
+    below section on {{OS Limits}}). When setting this value, please remember
+    that you will need space in memory for other things as well, such as the
+    DataNode and application JVM heaps and the operating system page cache.
 
 *** Optional
 
@@ -341,3 +339,6 @@ Centralized Cache Management in HDFS
   "unlimited," indicating that there is no limit.  Note that it's typical for
   <<<ulimit -l>>> to output the memory lock limit in KB, but
   dfs.datanode.max.locked.memory must be specified in bytes.
+
+  This information does not apply to deployments on Windows.  Windows has no
+  direct equivalent of <<<ulimit -l>>>.

Modified: hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/site/apt/Federation.apt.vm
URL: http://svn.apache.org/viewvc/hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/site/apt/Federation.apt.vm?rev=1619012&r1=1619011&r2=1619012&view=diff
==============================================================================
--- hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/site/apt/Federation.apt.vm (original)
+++ hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/site/apt/Federation.apt.vm Tue Aug 19 23:49:39 2014
@@ -287,13 +287,14 @@ HDFS Federation
 
   Policy could be:
 
-  * <<<node>>> - this is the <default> policy. This balances the storage at 
+  * <<<datanode>>> - this is the <default> policy. This balances the storage at 
     the datanode level. This is similar to balancing policy from prior releases.
 
   * <<<blockpool>>> - this balances the storage at the block pool level. 
     Balancing at block pool level balances storage at the datanode level also.
 
-  Note that Balander only balances the data and does not balance the namespace.
+  Note that Balancer only balances the data and does not balance the namespace.
+  For the complete command usage, see {{{../hadoop-common/CommandsManual.html#balancer}balancer}}.
 
 ** Decommissioning
 

Modified: hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/site/apt/HDFSHighAvailabilityWithNFS.apt.vm
URL: http://svn.apache.org/viewvc/hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/site/apt/HDFSHighAvailabilityWithNFS.apt.vm?rev=1619012&r1=1619011&r2=1619012&view=diff
==============================================================================
--- hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/site/apt/HDFSHighAvailabilityWithNFS.apt.vm (original)
+++ hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/site/apt/HDFSHighAvailabilityWithNFS.apt.vm Tue Aug 19 23:49:39 2014
@@ -838,41 +838,22 @@ digest:hdfs-zkfcs:vlUvLnd8MlacsE80rDuu6O
 
      $ mvn clean package -Pdist
 
-     This will generate a jar with the BookKeeperJournalManager, all the dependencies
-     needed by the journal manager,
+     This will generate a jar with the BookKeeperJournalManager, 
      hadoop-hdfs/src/contrib/bkjournal/target/hadoop-hdfs-bkjournal-<VERSION>.jar
 
-     Note that the -Pdist part of the build command is important, as otherwise
-     the dependencies would not be packaged in the jar. The dependencies included in
-     the jar are {{{http://maven.apache.org/plugins/maven-shade-plugin/}shaded}} to
-     avoid conflicts with other dependencies of the NameNode.
+     Note that the -Pdist part of the build command is important, this would
+     copy the dependent bookkeeper-server jar under 
+     hadoop-hdfs/src/contrib/bkjournal/target/lib.
 
  *** <<Putting the BookKeeperJournalManager in the NameNode classpath>>
 
-    To run a HDFS namenode using BookKeeper as a backend, copy the bkjournal
-    jar, generated above, into the lib directory of hdfs. In the standard 
-    distribution of HDFS, this is at $HADOOP_HDFS_HOME/share/hadoop/hdfs/lib/
+    To run a HDFS namenode using BookKeeper as a backend, copy the bkjournal and
+    bookkeeper-server jar, mentioned above, into the lib directory of hdfs. In the 
+    standard distribution of HDFS, this is at $HADOOP_HDFS_HOME/share/hadoop/hdfs/lib/
 
     cp hadoop-hdfs/src/contrib/bkjournal/target/hadoop-hdfs-bkjournal-<VERSION>.jar $HADOOP_HDFS_HOME/share/hadoop/hdfs/lib/
 
  *** <<Current limitations>> 
 
-      1) NameNode format command will not format the BookKeeper data automatically. 
-         We have to clean the data manually from BookKeeper cluster 
-         and create the /ledgers/available path in Zookeeper. 
-----
-$ zkCli.sh create /ledgers 0
-$ zkCli.sh create /ledgers/available 0
-----
-         Note:
-          bookkeeper://zk1:2181;zk2:2181;zk3:2181/hdfsjournal
-          The final part /hdfsjournal specifies the znode in zookeeper where
-          ledger metadata will be stored. Administrators may set this to anything
-          they wish.
-
-      2) Security in BookKeeper. BookKeeper does not support SASL nor SSL for
-         connections between the NameNode and BookKeeper storage nodes.
-
-      3) Auto-Recovery of storage node failures. Work inprogress 
-      {{{https://issues.apache.org/jira/browse/BOOKKEEPER-237 }BOOKKEEPER-237}}.
-         Currently we have the tools to manually recover the data from failed storage nodes.
+      1) Security in BookKeeper. BookKeeper does not support SASL nor SSL for
+         connections between the NameNode and BookKeeper storage nodes.
\ No newline at end of file

Modified: hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/site/apt/HDFSHighAvailabilityWithQJM.apt.vm
URL: http://svn.apache.org/viewvc/hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/site/apt/HDFSHighAvailabilityWithQJM.apt.vm?rev=1619012&r1=1619011&r2=1619012&view=diff
==============================================================================
--- hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/site/apt/HDFSHighAvailabilityWithQJM.apt.vm (original)
+++ hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/site/apt/HDFSHighAvailabilityWithQJM.apt.vm Tue Aug 19 23:49:39 2014
@@ -416,8 +416,8 @@ HDFS High Availability Using the Quorum 
 
   After all of the necessary configuration options have been set, you must
   start the JournalNode daemons on the set of machines where they will run. This
-  can be done by running the command "<hdfs-daemon.sh journalnode>" and waiting
-  for the daemon to start on each of the relevant machines.
+  can be done by running the command "<hadoop-daemon.sh start journalnode>" and
+  waiting for the daemon to start on each of the relevant machines.
 
   Once the JournalNodes have been started, one must initially synchronize the
   two HA NameNodes' on-disk metadata.

Modified: hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/site/apt/HdfsImageViewer.apt.vm
URL: http://svn.apache.org/viewvc/hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/site/apt/HdfsImageViewer.apt.vm?rev=1619012&r1=1619011&r2=1619012&view=diff
==============================================================================
--- hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/site/apt/HdfsImageViewer.apt.vm (original)
+++ hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/site/apt/HdfsImageViewer.apt.vm Tue Aug 19 23:49:39 2014
@@ -23,56 +23,29 @@ Offline Image Viewer Guide
 * Overview
 
    The Offline Image Viewer is a tool to dump the contents of hdfs fsimage
-   files to human-readable formats in order to allow offline analysis and
-   examination of an Hadoop cluster's namespace. The tool is able to
-   process very large image files relatively quickly, converting them to
-   one of several output formats. The tool handles the layout formats that
-   were included with Hadoop versions 16 and up. If the tool is not able
-   to process an image file, it will exit cleanly. The Offline Image
-   Viewer does not require an Hadoop cluster to be running; it is entirely
-   offline in its operation.
+   files to a human-readable format and provide read-only WebHDFS API
+   in order to allow offline analysis and examination of an Hadoop cluster's
+   namespace. The tool is able to process very large image files relatively
+   quickly. The tool handles the layout formats that were included with Hadoop
+   versions 2.4 and up. If you want to handle older layout formats, you can
+   use the Offline Image Viewer of Hadoop 2.3.
+   If the tool is not able to process an image file, it will exit cleanly.
+   The Offline Image Viewer does not require a Hadoop cluster to be running;
+   it is entirely offline in its operation.
 
    The Offline Image Viewer provides several output processors:
 
-   [[1]] Ls is the default output processor. It closely mimics the format of
-      the lsr command. It includes the same fields, in the same order, as
-      lsr : directory or file flag, permissions, replication, owner,
-      group, file size, modification date, and full path. Unlike the lsr
-      command, the root path is included. One important difference
-      between the output of the lsr command this processor, is that this
-      output is not sorted by directory name and contents. Rather, the
-      files are listed in the order in which they are stored in the
-      fsimage file. Therefore, it is not possible to directly compare the
-      output of the lsr command this this tool. The Ls processor uses
-      information contained within the Inode blocks to calculate file
-      sizes and ignores the -skipBlocks option.
-
-   [[2]] Indented provides a more complete view of the fsimage's contents,
-      including all of the information included in the image, such as
-      image version, generation stamp and inode- and block-specific
-      listings. This processor uses indentation to organize the output
-      into a hierarchal manner. The lsr format is suitable for easy human
-      comprehension.
-
-   [[3]] Delimited provides one file per line consisting of the path,
-      replication, modification time, access time, block size, number of
-      blocks, file size, namespace quota, diskspace quota, permissions,
-      username and group name. If run against an fsimage that does not
-      contain any of these fields, the field's column will be included,
-      but no data recorded. The default record delimiter is a tab, but
-      this may be changed via the -delimiter command line argument. This
-      processor is designed to create output that is easily analyzed by
-      other tools, such as {{{http://pig.apache.org}Apache Pig}}. See
-      the {{Analyzing Results}} section for further information on using
-      this processor to analyze the contents of fsimage files.
+   [[1]] Web is the default output processor. It launches a HTTP server
+      that exposes read-only WebHDFS API. Users can investigate the namespace
+      interactively by using HTTP REST API.
 
-   [[4]] XML creates an XML document of the fsimage and includes all of the
+   [[2]] XML creates an XML document of the fsimage and includes all of the
       information within the fsimage, similar to the lsr processor. The
       output of this processor is amenable to automated processing and
       analysis with XML tools. Due to the verbosity of the XML syntax,
       this processor will also generate the largest amount of output.
 
-   [[5]] FileDistribution is the tool for analyzing file sizes in the
+   [[3]] FileDistribution is the tool for analyzing file sizes in the
       namespace image. In order to run the tool one should define a range
       of integers [0, maxSize] by specifying maxSize and a step. The
       range of integers is divided into segments of size step: [0, s[1],
@@ -86,105 +59,93 @@ Offline Image Viewer Guide
 
 * Usage
 
-** Basic
+** Web Processor
 
-   The simplest usage of the Offline Image Viewer is to provide just an
-   input and output file, via the -i and -o command-line switches:
+   Web processor launches a HTTP server which exposes read-only WebHDFS API.
+   Users can specify the address to listen by -addr option (default by
+   localhost:5978).
 
 ----
-   bash$ bin/hdfs oiv -i fsimage -o fsimage.txt
+   bash$ bin/hdfs oiv -i fsimage
+   14/04/07 13:25:14 INFO offlineImageViewer.WebImageViewer: WebImageViewer
+   started. Listening on /127.0.0.1:5978. Press Ctrl+C to stop the viewer.
 ----
 
-   This will create a file named fsimage.txt in the current directory
-   using the Ls output processor. For very large image files, this process
-   may take several minutes.
+   Users can access the viewer and get the information of the fsimage by
+   the following shell command:
 
-   One can specify which output processor via the command-line switch -p.
-   For instance:
+----
+   bash$ bin/hdfs dfs -ls webhdfs://127.0.0.1:5978/
+   Found 2 items
+   drwxrwx---   - root supergroup          0 2014-03-26 20:16 webhdfs://127.0.0.1:5978/tmp
+   drwxr-xr-x   - root supergroup          0 2014-03-31 14:08 webhdfs://127.0.0.1:5978/user
+----
+
+   To get the information of all the files and directories, you can simply use
+   the following command:
 
 ----
-   bash$ bin/hdfs oiv -i fsimage -o fsimage.xml -p XML
+   bash$ bin/hdfs dfs -ls -R webhdfs://127.0.0.1:5978/
 ----
 
-   or
+   Users can also get JSON formatted FileStatuses via HTTP REST API.
 
 ----
-   bash$ bin/hdfs oiv -i fsimage -o fsimage.txt -p Indented
+   bash$ curl -i http://127.0.0.1:5978/webhdfs/v1/?op=liststatus
+   HTTP/1.1 200 OK
+   Content-Type: application/json
+   Content-Length: 252
+
+   {"FileStatuses":{"FileStatus":[
+   {"fileId":16386,"accessTime":0,"replication":0,"owner":"theuser","length":0,"permission":"755","blockSize":0,"modificationTime":1392772497282,"type":"DIRECTORY","group":"supergroup","childrenNum":1,"pathSuffix":"user"}
+   ]}}
 ----
 
-   This will run the tool using either the XML or Indented output
-   processor, respectively.
+   The Web processor now supports the following operations:
 
-   One command-line option worth considering is -skipBlocks, which
-   prevents the tool from explicitly enumerating all of the blocks that
-   make up a file in the namespace. This is useful for file systems that
-   have very large files. Enabling this option can significantly decrease
-   the size of the resulting output, as individual blocks are not
-   included. Note, however, that the Ls processor needs to enumerate the
-   blocks and so overrides this option.
+   * {{{./WebHDFS.html#List_a_Directory}LISTSTATUS}}
 
-Example
+   * {{{./WebHDFS.html#Status_of_a_FileDirectory}GETFILESTATUS}}
 
-   Consider the following contrived namespace:
+   * {{{./WebHDFS.html#Get_ACL_Status}GETACLSTATUS}}
 
-----
-   drwxr-xr-x   - theuser supergroup          0 2009-03-16 21:17 /anotherDir
-   -rw-r--r--   3 theuser supergroup  286631664 2009-03-16 21:15 /anotherDir/biggerfile
-   -rw-r--r--   3 theuser supergroup       8754 2009-03-16 21:17 /anotherDir/smallFile
-   drwxr-xr-x   - theuser supergroup          0 2009-03-16 21:11 /mapredsystem
-   drwxr-xr-x   - theuser supergroup          0 2009-03-16 21:11 /mapredsystem/theuser
-   drwxr-xr-x   - theuser supergroup          0 2009-03-16 21:11 /mapredsystem/theuser/mapredsystem
-   drwx-wx-wx   - theuser supergroup          0 2009-03-16 21:11 /mapredsystem/theuser/mapredsystem/ip.redacted.com
-   drwxr-xr-x   - theuser supergroup          0 2009-03-16 21:12 /one
-   drwxr-xr-x   - theuser supergroup          0 2009-03-16 21:12 /one/two
-   drwxr-xr-x   - theuser supergroup          0 2009-03-16 21:16 /user
-   drwxr-xr-x   - theuser supergroup          0 2009-03-16 21:19 /user/theuser
-----
+** XML Processor
 
-   Applying the Offline Image Processor against this file with default
-   options would result in the following output:
+   XML Processor is used to dump all the contents in the fsimage. Users can
+   specify input and output file via -i and -o command-line.
 
 ----
-   machine:hadoop-0.21.0-dev theuser$ bin/hdfs oiv -i fsimagedemo -o fsimage.txt
-
-   drwxr-xr-x  -   theuser supergroup            0 2009-03-16 14:16 /
-   drwxr-xr-x  -   theuser supergroup            0 2009-03-16 14:17 /anotherDir
-   drwxr-xr-x  -   theuser supergroup            0 2009-03-16 14:11 /mapredsystem
-   drwxr-xr-x  -   theuser supergroup            0 2009-03-16 14:12 /one
-   drwxr-xr-x  -   theuser supergroup            0 2009-03-16 14:16 /user
-   -rw-r--r--  3   theuser supergroup    286631664 2009-03-16 14:15 /anotherDir/biggerfile
-   -rw-r--r--  3   theuser supergroup         8754 2009-03-16 14:17 /anotherDir/smallFile
-   drwxr-xr-x  -   theuser supergroup            0 2009-03-16 14:11 /mapredsystem/theuser
-   drwxr-xr-x  -   theuser supergroup            0 2009-03-16 14:11 /mapredsystem/theuser/mapredsystem
-   drwx-wx-wx  -   theuser supergroup            0 2009-03-16 14:11 /mapredsystem/theuser/mapredsystem/ip.redacted.com
-   drwxr-xr-x  -   theuser supergroup            0 2009-03-16 14:12 /one/two
-   drwxr-xr-x  -   theuser supergroup            0 2009-03-16 14:19 /user/theuser
+   bash$ bin/hdfs oiv -p XML -i fsimage -o fsimage.xml
 ----
 
-   Similarly, applying the Indented processor would generate output that
-   begins with:
+   This will create a file named fsimage.xml contains all the information in
+   the fsimage. For very large image files, this process may take several
+   minutes.
 
-----
-   machine:hadoop-0.21.0-dev theuser$ bin/hdfs oiv -i fsimagedemo -p Indented -o fsimage.txt
+   Applying the Offline Image Viewer with XML processor would result in the
+   following output:
 
-   FSImage
-     ImageVersion = -19
-     NamespaceID = 2109123098
-     GenerationStamp = 1003
-     INodes [NumInodes = 12]
-       Inode
-         INodePath =
-         Replication = 0
-         ModificationTime = 2009-03-16 14:16
-         AccessTime = 1969-12-31 16:00
-         BlockSize = 0
-         Blocks [NumBlocks = -1]
-         NSQuota = 2147483647
-         DSQuota = -1
-         Permissions
-           Username = theuser
-           GroupName = supergroup
-           PermString = rwxr-xr-x
+----
+   <?xml version="1.0"?>
+   <fsimage>
+   <NameSection>
+     <genstampV1>1000</genstampV1>
+     <genstampV2>1002</genstampV2>
+     <genstampV1Limit>0</genstampV1Limit>
+     <lastAllocatedBlockId>1073741826</lastAllocatedBlockId>
+     <txid>37</txid>
+   </NameSection>
+   <INodeSection>
+     <lastInodeId>16400</lastInodeId>
+     <inode>
+       <id>16385</id>
+       <type>DIRECTORY</type>
+       <name></name>
+       <mtime>1392772497282</mtime>
+       <permission>theuser:supergroup:rwxr-xr-x</permission>
+       <nsquota>9223372036854775807</nsquota>
+       <dsquota>-1</dsquota>
+     </inode>
    ...remaining output omitted...
 ----
 
@@ -193,30 +154,32 @@ Example
 *-----------------------:-----------------------------------+
 | <<Flag>>              | <<Description>>                   |
 *-----------------------:-----------------------------------+
-| <<<-i>>>\|<<<--inputFile>>> <input file> | Specify the input fsimage file to
-|                       | process. Required.
+| <<<-i>>>\|<<<--inputFile>>> <input file> | Specify the input fsimage file
+|                       | to process. Required.
 *-----------------------:-----------------------------------+
-| <<<-o>>>\|<<<--outputFile>>> <output file> | Specify the output filename, if the
-|                       | specified output processor generates one. If the specified file already
-|                       | exists, it is silently overwritten. Required.
+| <<<-o>>>\|<<<--outputFile>>> <output file> | Specify the output filename,
+|                       | if the specified output processor generates one. If
+|                       | the specified file already exists, it is silently
+|                       | overwritten. (output to stdout by default)
+*-----------------------:-----------------------------------+
+| <<<-p>>>\|<<<--processor>>> <processor> | Specify the image processor to
+|                       | apply against the image file. Currently valid options
+|                       | are Web (default), XML and FileDistribution.
+*-----------------------:-----------------------------------+
+| <<<-addr>>> <address> | Specify the address(host:port) to listen.
+|                       | (localhost:5978 by default). This option is used with
+|                       | Web processor.
+*-----------------------:-----------------------------------+
+| <<<-maxSize>>> <size> | Specify the range [0, maxSize] of file sizes to be
+|                       | analyzed in bytes (128GB by default). This option is
+|                       | used with FileDistribution processor.
+*-----------------------:-----------------------------------+
+| <<<-step>>> <size>    | Specify the granularity of the distribution in bytes
+|                       | (2MB by default). This option is used with
+|                       | FileDistribution processor.
 *-----------------------:-----------------------------------+
-| <<<-p>>>\|<<<--processor>>> <processor> | Specify the image processor to apply
-|                       | against the image file. Currently valid options are Ls (default), XML
-|                       | and Indented..
-*-----------------------:-----------------------------------+
-| <<<-skipBlocks>>>     | Do not enumerate individual blocks within files. This may
-|                       | save processing time and outfile file space on namespaces with very
-|                       | large files. The Ls processor reads the blocks to correctly determine
-|                       | file sizes and ignores this option.
-*-----------------------:-----------------------------------+
-| <<<-printToScreen>>>  | Pipe output of processor to console as well as specified
-|                       | file. On extremely large namespaces, this may increase processing time
-|                       | by an order of magnitude.
-*-----------------------:-----------------------------------+
-| <<<-delimiter>>> <arg>| When used in conjunction with the Delimited processor,
-|                       | replaces the default tab delimiter with the string specified by arg.
-*-----------------------:-----------------------------------+
-| <<<-h>>>\|<<<--help>>>| Display the tool usage and help information and exit.
+| <<<-h>>>\|<<<--help>>>| Display the tool usage and help information and
+|                       | exit.
 *-----------------------:-----------------------------------+
 
 * Analyzing Results
@@ -224,193 +187,4 @@ Example
    The Offline Image Viewer makes it easy to gather large amounts of data
    about the hdfs namespace. This information can then be used to explore
    file system usage patterns or find specific files that match arbitrary
-   criteria, along with other types of namespace analysis. The Delimited
-   image processor in particular creates output that is amenable to
-   further processing by tools such as [38]Apache Pig. Pig provides a
-   particularly good choice for analyzing these data as it is able to deal
-   with the output generated from a small fsimage but also scales up to
-   consume data from extremely large file systems.
-
-   The Delimited image processor generates lines of text separated, by
-   default, by tabs and includes all of the fields that are common between
-   constructed files and files that were still under constructed when the
-   fsimage was generated. Examples scripts are provided demonstrating how
-   to use this output to accomplish three tasks: determine the number of
-   files each user has created on the file system, find files were created
-   but have not accessed, and find probable duplicates of large files by
-   comparing the size of each file.
-
-   Each of the following scripts assumes you have generated an output file
-   using the Delimited processor named foo and will be storing the results
-   of the Pig analysis in a file named results.
-
-** Total Number of Files for Each User
-
-   This script processes each path within the namespace, groups them by
-   the file owner and determines the total number of files each user owns.
-
-----
-      numFilesOfEachUser.pig:
-   -- This script determines the total number of files each user has in
-   -- the namespace. Its output is of the form:
-   --   username, totalNumFiles
-
-   -- Load all of the fields from the file
-   A = LOAD '$inputFile' USING PigStorage('\t') AS (path:chararray,
-                                                    replication:int,
-                                                    modTime:chararray,
-                                                    accessTime:chararray,
-                                                    blockSize:long,
-                                                    numBlocks:int,
-                                                    fileSize:long,
-                                                    NamespaceQuota:int,
-                                                    DiskspaceQuota:int,
-                                                    perms:chararray,
-                                                    username:chararray,
-                                                    groupname:chararray);
-
-
-   -- Grab just the path and username
-   B = FOREACH A GENERATE path, username;
-
-   -- Generate the sum of the number of paths for each user
-   C = FOREACH (GROUP B BY username) GENERATE group, COUNT(B.path);
-
-   -- Save results
-   STORE C INTO '$outputFile';
-----
-
-   This script can be run against pig with the following command:
-
-----
-   bin/pig -x local -param inputFile=../foo -param outputFile=../results ../numFilesOfEachUser.pig
-----
-
-   The output file's content will be similar to that below:
-
-----
-   bart 1
-   lisa 16
-   homer 28
-   marge 2456
-----
-
-** Files That Have Never Been Accessed
-
-   This script finds files that were created but whose access times were
-   never changed, meaning they were never opened or viewed.
-
-----
-      neverAccessed.pig:
-   -- This script generates a list of files that were created but never
-   -- accessed, based on their AccessTime
-
-   -- Load all of the fields from the file
-   A = LOAD '$inputFile' USING PigStorage('\t') AS (path:chararray,
-                                                    replication:int,
-                                                    modTime:chararray,
-                                                    accessTime:chararray,
-                                                    blockSize:long,
-                                                    numBlocks:int,
-                                                    fileSize:long,
-                                                    NamespaceQuota:int,
-                                                    DiskspaceQuota:int,
-                                                    perms:chararray,
-                                                    username:chararray,
-                                                    groupname:chararray);
-
-   -- Grab just the path and last time the file was accessed
-   B = FOREACH A GENERATE path, accessTime;
-
-   -- Drop all the paths that don't have the default assigned last-access time
-   C = FILTER B BY accessTime == '1969-12-31 16:00';
-
-   -- Drop the accessTimes, since they're all the same
-   D = FOREACH C GENERATE path;
-
-   -- Save results
-   STORE D INTO '$outputFile';
-----
-
-   This script can be run against pig with the following command and its
-   output file's content will be a list of files that were created but
-   never viewed afterwards.
-
-----
-   bin/pig -x local -param inputFile=../foo -param outputFile=../results ../neverAccessed.pig
-----
-
-** Probable Duplicated Files Based on File Size
-
-   This script groups files together based on their size, drops any that
-   are of less than 100mb and returns a list of the file size, number of
-   files found and a tuple of the file paths. This can be used to find
-   likely duplicates within the filesystem namespace.
-
-----
-      probableDuplicates.pig:
-   -- This script finds probable duplicate files greater than 100 MB by
-   -- grouping together files based on their byte size. Files of this size
-   -- with exactly the same number of bytes can be considered probable
-   -- duplicates, but should be checked further, either by comparing the
-   -- contents directly or by another proxy, such as a hash of the contents.
-   -- The scripts output is of the type:
-   --    fileSize numProbableDuplicates {(probableDup1), (probableDup2)}
-
-   -- Load all of the fields from the file
-   A = LOAD '$inputFile' USING PigStorage('\t') AS (path:chararray,
-                                                    replication:int,
-                                                    modTime:chararray,
-                                                    accessTime:chararray,
-                                                    blockSize:long,
-                                                    numBlocks:int,
-                                                    fileSize:long,
-                                                    NamespaceQuota:int,
-                                                    DiskspaceQuota:int,
-                                                    perms:chararray,
-                                                    username:chararray,
-                                                    groupname:chararray);
-
-   -- Grab the pathname and filesize
-   B = FOREACH A generate path, fileSize;
-
-   -- Drop files smaller than 100 MB
-   C = FILTER B by fileSize > 100L  * 1024L * 1024L;
-
-   -- Gather all the files of the same byte size
-   D = GROUP C by fileSize;
-
-   -- Generate path, num of duplicates, list of duplicates
-   E = FOREACH D generate group AS fileSize, COUNT(C) as numDupes, C.path AS files;
-
-   -- Drop all the files where there are only one of them
-   F = FILTER E by numDupes > 1L;
-
-   -- Sort by the size of the files
-   G = ORDER F by fileSize;
-
-   -- Save results
-   STORE G INTO '$outputFile';
-----
-
-   This script can be run against pig with the following command:
-
-----
-   bin/pig -x local -param inputFile=../foo -param outputFile=../results ../probableDuplicates.pig
-----
-
-   The output file's content will be similar to that below:
-
-----
-   1077288632 2 {(/user/tennant/work1/part-00501),(/user/tennant/work1/part-00993)}
-   1077288664 4 {(/user/tennant/work0/part-00567),(/user/tennant/work0/part-03980),(/user/tennant/work1/part-00725),(/user/eccelston/output/part-03395)}
-   1077288668 3 {(/user/tennant/work0/part-03705),(/user/tennant/work0/part-04242),(/user/tennant/work1/part-03839)}
-   1077288698 2 {(/user/tennant/work0/part-00435),(/user/eccelston/output/part-01382)}
-   1077288702 2 {(/user/tennant/work0/part-03864),(/user/eccelston/output/part-03234)}
-----
-
-   Each line includes the file size in bytes that was found to be
-   duplicated, the number of duplicates found, and a list of the
-   duplicated paths. Files less than 100MB are ignored, providing a
-   reasonable likelihood that files of these exact sizes may be
-   duplicates.
+   criteria, along with other types of namespace analysis.

Modified: hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/site/apt/HdfsNfsGateway.apt.vm
URL: http://svn.apache.org/viewvc/hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/site/apt/HdfsNfsGateway.apt.vm?rev=1619012&r1=1619011&r2=1619012&view=diff
==============================================================================
--- hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/site/apt/HdfsNfsGateway.apt.vm (original)
+++ hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/site/apt/HdfsNfsGateway.apt.vm Tue Aug 19 23:49:39 2014
@@ -19,8 +19,6 @@
 
 HDFS NFS Gateway
 
-  \[ {{{./index.html}Go Back}} \]
-
 %{toc|section=1|fromDepth=0}
 
 * {Overview}
@@ -46,18 +44,24 @@ HDFS NFS Gateway
 
 * {Configuration}
 
-   The user running the NFS-gateway must be able to proxy all the users using the NFS mounts. 
-   For instance, if user 'nfsserver' is running the gateway, and users belonging to the groups 'nfs-users1'
-   and 'nfs-users2' use the NFS mounts, then in core-site.xml of the namenode, the following must be set
-   (NOTE: replace 'nfsserver' with the user name starting the gateway in your cluster):
+   The NFS-gateway uses proxy user to proxy all the users accessing the NFS mounts. 
+   In non-secure mode, the user running the gateway is the proxy user, while in secure mode the
+   user in Kerberos keytab is the proxy user. Suppose the proxy user is 'nfsserver'
+   and users belonging to the groups 'users-group1'
+   and 'users-group2' use the NFS mounts, then in core-site.xml of the NameNode, the following
+   two properities must be set and only NameNode needs restart after the configuration change
+   (NOTE: replace the string 'nfsserver' with the proxy user name in your cluster):
 
 ----
 <property>
   <name>hadoop.proxyuser.nfsserver.groups</name>
-  <value>nfs-users1,nfs-users2</value>
+  <value>root,users-group1,users-group2</value>
   <description>
-         The 'nfsserver' user is allowed to proxy all members of the 'nfs-users1' and 
-         'nfs-users2' groups. Set this to '*' to allow nfsserver user to proxy any group.
+         The 'nfsserver' user is allowed to proxy all members of the 'users-group1' and 
+         'users-group2' groups. Note that in most cases you will need to include the
+         group "root" because the user "root" (which usually belonges to "root" group) will
+         generally be the user that initially executes the mount on the NFS client system. 
+         Set this to '*' to allow nfsserver user to proxy any group.
   </description>
 </property>
 ----
@@ -74,24 +78,47 @@ HDFS NFS Gateway
 ----
 
    The above are the only required configuration for the NFS gateway in non-secure mode. For Kerberized
-   hadoop clusters, the following configurations need to be added to hdfs-site.xml:
+   hadoop clusters, the following configurations need to be added to hdfs-site.xml for the gateway (NOTE: replace 
+   string "nfsserver" with the proxy user name and ensure the user contained in the keytab is
+   also the same proxy user):
 
 ----
   <property>
-    <name>dfs.nfs.keytab.file</name>
+    <name>nfs.keytab.file</name>
     <value>/etc/hadoop/conf/nfsserver.keytab</value> <!-- path to the nfs gateway keytab -->
   </property>
 ----
 
 ----
   <property>
-    <name>dfs.nfs.kerberos.principal</name>
+    <name>nfs.kerberos.principal</name>
     <value>nfsserver/_HOST@YOUR-REALM.COM</value>
   </property>
 ----
+  
+   The rest of the NFS gateway configurations are optional for both secure and non-secure mode.
+
+   The AIX NFS client has a {{{https://issues.apache.org/jira/browse/HDFS-6549}few known issues}}
+   that prevent it from working correctly by default with the HDFS NFS
+   Gateway. If you want to be able to access the HDFS NFS Gateway from AIX, you
+   should set the following configuration setting to enable work-arounds for these
+   issues:
+
+----
+<property>
+  <name>nfs.aix.compatibility.mode.enabled</name>
+  <value>true</value>
+</property>
+----
+
+   Note that regular, non-AIX clients should NOT enable AIX compatibility mode.
+   The work-arounds implemented by AIX compatibility mode effectively disable
+   safeguards to ensure that listing of directory contents via NFS returns
+   consistent results, and that all data sent to the NFS server can be assured to
+   have been committed.
 
    It's strongly recommended for the users to update a few configuration properties based on their use
-   cases. All the related configuration properties can be added or updated in hdfs-site.xml.
+   cases. All the following configuration properties can be added or updated in hdfs-site.xml.
   
    * If the client mounts the export with access time update allowed, make sure the following 
     property is not disabled in the configuration file. Only NameNode needs to restart after 
@@ -123,41 +150,11 @@ HDFS NFS Gateway
 
 ----
   <property>    
-    <name>dfs.nfs3.dump.dir</name>
+    <name>nfs.dump.dir</name>
     <value>/tmp/.hdfs-nfs</value>
   </property>
 ---- 
 
-   * For optimal performance, it is recommended that rtmax be updated to
-     1MB. However, note that this 1MB is a per client allocation, and not
-     from a shared memory pool, and therefore a larger value may adversely 
-     affect small reads, consuming a lot of memory. The maximum value of 
-     this property is 1MB.
-
-----
-<property>
-  <name>dfs.nfs.rtmax</name>
-  <value>1048576</value>
-  <description>This is the maximum size in bytes of a READ request
-    supported by the NFS gateway. If you change this, make sure you
-    also update the nfs mount's rsize(add rsize= # of bytes to the 
-    mount directive).
-  </description>
-</property>
-----
-
-----
-<property>
-  <name>dfs.nfs.wtmax</name>
-  <value>65536</value>
-  <description>This is the maximum size in bytes of a WRITE request
-    supported by the NFS gateway. If you change this, make sure you
-    also update the nfs mount's wsize(add wsize= # of bytes to the 
-    mount directive).
-  </description>
-</property>
-----
-
   * By default, the export can be mounted by any client. To better control the access,
     users can update the following property. The value string contains machine name and
     access privilege, separated by whitespace
@@ -169,7 +166,7 @@ HDFS NFS Gateway
 
 ----
 <property>
-  <name>dfs.nfs.exports.allowed.hosts</name>
+  <name>nfs.exports.allowed.hosts</name>
   <value>* rw</value>
 </property>
 ----
@@ -221,8 +218,10 @@ HDFS NFS Gateway
 
    [[3]] Start mountd and nfsd.
    
-     No root privileges are required for this command. However, ensure that the user starting
-     the Hadoop cluster and the user starting the NFS gateway are same.
+     No root privileges are required for this command. In non-secure mode, the NFS gateway
+     should be started by the proxy user mentioned at the beginning of this user guide. 
+     While in secure mode, any user can start NFS gateway 
+     as long as the user has read access to the Kerberos keytab defined in "nfs.keytab.file".
 
 -------------------------
      hadoop nfs3
@@ -243,6 +242,19 @@ HDFS NFS Gateway
       hadoop-daemon.sh stop portmap
 -------------------------
 
+  Optionally, you can forgo running the Hadoop-provided portmap daemon and
+  instead use the system portmap daemon on all operating systems if you start the
+  NFS Gateway as root. This will allow the HDFS NFS Gateway to work around the
+  aforementioned bug and still register using the system portmap daemon. To do
+  so, just start the NFS gateway daemon as you normally would, but make sure to
+  do so as the "root" user, and also set the "HADOOP_PRIVILEGED_NFS_USER"
+  environment variable to an unprivileged user. In this mode the NFS Gateway will
+  start as root to perform its initial registration with the system portmap, and
+  then will drop privileges back to the user specified by the
+  HADOOP_PRIVILEGED_NFS_USER afterward and for the rest of the duration of the
+  lifetime of the NFS Gateway process. Note that if you choose this route, you
+  should skip steps 1 and 2 above.
+
 
 * {Verify validity of NFS related services}
 
@@ -305,11 +317,30 @@ HDFS NFS Gateway
   The users can mount the HDFS namespace as shown below:
 
 -------------------------------------------------------------------  
-       mount -t nfs -o vers=3,proto=tcp,nolock $server:/  $mount_point
+       mount -t nfs -o vers=3,proto=tcp,nolock,noacl $server:/  $mount_point
 -------------------------------------------------------------------
 
   Then the users can access HDFS as part of the local file system except that, 
-  hard link and random write are not supported yet.
+  hard link and random write are not supported yet. To optimize the performance
+  of large file I/O, one can increase the NFS transfer size(rsize and wsize) during mount.
+  By default, NFS gateway supports 1MB as the maximum transfer size. For larger data
+  transfer size, one needs to update "nfs.rtmax" and "nfs.rtmax" in hdfs-site.xml.
+
+* {Allow mounts from unprivileged clients}
+
+  In environments where root access on client machines is not generally
+  available, some measure of security can be obtained by ensuring that only NFS
+  clients originating from privileged ports can connect to the NFS server. This
+  feature is referred to as "port monitoring." This feature is not enabled by default
+  in the HDFS NFS Gateway, but can be optionally enabled by setting the
+  following config in hdfs-site.xml on the NFS Gateway machine:
+
+-------------------------------------------------------------------
+<property>
+  <name>nfs.port.monitoring.disabled</name>
+  <value>false</value>
+</property>
+-------------------------------------------------------------------
 
 * {User authentication and mapping}
 
@@ -325,8 +356,21 @@ HDFS NFS Gateway
   The system administrator must ensure that the user on NFS client host has the same
   name and UID as that on the NFS gateway host. This is usually not a problem if
   the same user management system (e.g., LDAP/NIS) is used to create and deploy users on
-  HDFS nodes and NFS client node. In case the user account is created manually in different hosts, one might need to 
+  HDFS nodes and NFS client node. In case the user account is created manually on different hosts, one might need to 
   modify UID (e.g., do "usermod -u 123 myusername") on either NFS client or NFS gateway host
   in order to make it the same on both sides. More technical details of RPC AUTH_UNIX can be found
   in {{{http://tools.ietf.org/html/rfc1057}RPC specification}}.
 
+  Optionally, the system administrator can configure a custom static mapping
+  file in the event one wishes to access the HDFS NFS Gateway from a system with
+  a completely disparate set of UIDs/GIDs. By default this file is located at
+  "/etc/nfs.map", but a custom location can be configured by setting the
+  "nfs.static.mapping.file" property to the path of the static mapping file.
+  The format of the static mapping file is similar to what is described in the
+  exports(5) manual page, but roughly it is:
+
+-------------------------
+# Mapping for clients accessing the NFS gateway
+uid 10 100 # Map the remote UID 10 the local UID 100
+gid 11 101 # Map the remote GID 11 to the local GID 101
+-------------------------

Modified: hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/site/apt/HdfsPermissionsGuide.apt.vm
URL: http://svn.apache.org/viewvc/hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/site/apt/HdfsPermissionsGuide.apt.vm?rev=1619012&r1=1619011&r2=1619012&view=diff
==============================================================================
--- hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/site/apt/HdfsPermissionsGuide.apt.vm (original)
+++ hadoop/common/branches/HADOOP-10388/hadoop-hdfs-project/hadoop-hdfs/src/site/apt/HdfsPermissionsGuide.apt.vm Tue Aug 19 23:49:39 2014
@@ -395,8 +395,8 @@ HDFS Permissions Guide
        permission checking is turned off, but all other behavior is
        unchanged. Switching from one parameter value to the other does not
        change the mode, owner or group of files or directories.
-       Regardless of whether permissions are on or off, chmod, chgrp and
-       chown always check permissions. These functions are only useful in
+       Regardless of whether permissions are on or off, chmod, chgrp, chown and
+       setfacl always check permissions. These functions are only useful in
        the permissions context, and so there is no backwards compatibility
        issue. Furthermore, this allows administrators to reliably set
        owners and permissions in advance of turning on regular permissions