Forráskód Böngészése

HADOOP-3248. Optimization of saveFSImage. Contributed by Dhruba.

git-svn-id: https://svn.apache.org/repos/asf/hadoop/core/trunk@653866 13f79535-47bb-0310-9956-ffa450edef68
Konstantin Shvachko 17 éve
szülő
commit
1c27228a02

+ 2 - 0
CHANGES.txt

@@ -100,6 +100,8 @@ Trunk (unreleased changes)
     On Linux DataNode takes 5 times less CPU while serving data. Results may
     vary on other platforms. (rangadi)
 
+    HADOOP-3248. Optimization of saveFSImage. (Dhruba via shv)
+
   BUG FIXES
 
     HADOOP-2905. 'fsck -move' triggers NPE in NameNode. 

+ 36 - 10
src/java/org/apache/hadoop/dfs/FSImage.java

@@ -37,8 +37,10 @@ import java.util.Iterator;
 import java.util.Properties;
 import java.util.Random;
 import java.lang.Math;
+import java.nio.ByteBuffer;
 
 import org.apache.hadoop.fs.permission.PermissionStatus;
+import org.apache.hadoop.fs.permission.FsPermission;
 import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.dfs.FSConstants.CheckpointStates;
 import org.apache.hadoop.dfs.FSConstants.StartupOption;
@@ -84,6 +86,13 @@ class FSImage extends Storage {
    */
   volatile private CheckpointStates ckptState = CheckpointStates.START; 
 
+  /**
+   * Used for saving the image to disk
+   */
+  static private final FsPermission fileperm = new FsPermission((short)0);
+  static private final byte[] separator = INode.string2Bytes("/");
+  static private byte[] byteStore = null;
+
   /**
    */
   FSImage() {
@@ -785,7 +794,7 @@ class FSImage extends Storage {
   /**
    * Save the contents of the FS image to the file.
    */
-  void saveFSImage(File newFile ) throws IOException {
+  void saveFSImage(File newFile) throws IOException {
     FSNamesystem fsNamesys = FSNamesystem.getFSNamesystem();
     FSDirectory fsDir = fsNamesys.dir;
     //
@@ -799,8 +808,12 @@ class FSImage extends Storage {
       out.writeInt(namespaceID);
       out.writeInt(fsDir.rootDir.numItemsInTree() - 1);
       out.writeLong(fsNamesys.getGenerationStamp());
-      saveImage("", fsDir.rootDir, out);
+      byteStore = new byte[4*FSConstants.MAX_PATH_LENGTH];
+      ByteBuffer strbuf = ByteBuffer.wrap(byteStore);
+      saveImage(strbuf, 0, fsDir.rootDir, out);
       fsNamesys.saveFilesUnderConstruction(out);
+      byteStore = null;
+      strbuf = null;
     } finally {
       out.close();
     }
@@ -875,13 +888,16 @@ class FSImage extends Storage {
   /**
    * Save file tree image starting from the given root.
    */
-  private static void saveImage(String parentPrefix, 
+  private static void saveImage(ByteBuffer parentPrefix, 
+                                int prefixLength,
                                 INode inode, 
                                 DataOutputStream out) throws IOException {
-    String fullName = "";
+    int newPrefixLength = prefixLength;
     if (inode.getParent() != null) {
-      fullName = parentPrefix + "/" + inode.getLocalName();
-      new UTF8(fullName).write(out);
+      parentPrefix.put(separator).put(inode.getLocalNameBytes());
+      newPrefixLength += separator.length + inode.getLocalNameBytes().length;
+      out.writeShort(newPrefixLength);
+      out.write(byteStore, 0, newPrefixLength);
       if (!inode.isDirectory()) {  // write file inode
         INodeFile fileINode = (INodeFile)inode;
         out.writeShort(fileINode.getReplication());
@@ -891,7 +907,11 @@ class FSImage extends Storage {
         out.writeInt(blocks.length);
         for (Block blk : blocks)
           blk.write(out);
-        fileINode.getPermissionStatus().write(out);
+        fileperm.fromShort(fileINode.getFsPermissionShort());
+        PermissionStatus.write(out, fileINode.getUserName(),
+                               fileINode.getGroupName(),
+                               fileperm);
+        parentPrefix.position(prefixLength);
         return;
       }
       // write directory inode
@@ -899,11 +919,17 @@ class FSImage extends Storage {
       out.writeLong(inode.getModificationTime());
       out.writeLong(0);   // preferred block size
       out.writeInt(-1);    // # of blocks
-      inode.getPermissionStatus().write(out);
+      fileperm.fromShort(inode.getFsPermissionShort());
+      PermissionStatus.write(out, inode.getUserName(),
+                             inode.getGroupName(),
+                             fileperm);
     }
-    for(INode child : ((INodeDirectory)inode).getChildren()) {
-      saveImage(fullName, child, out);
+    if (((INodeDirectory)inode).getChildrenRaw() != null) {
+      for(INode child : ((INodeDirectory)inode).getChildren()) {
+        saveImage(parentPrefix, newPrefixLength, child, out);
+      }
     }
+    parentPrefix.position(prefixLength);
   }
 
   void loadDatanodes(int version, DataInputStream in) throws IOException {

+ 14 - 0
src/java/org/apache/hadoop/dfs/INode.java

@@ -125,6 +125,9 @@ abstract class INode implements Comparable<byte[]> {
     return new FsPermission(
         (short)PermissionStatusFormat.MODE.retrieve(permission));
   }
+  protected short getFsPermissionShort() {
+    return (short)PermissionStatusFormat.MODE.retrieve(permission);
+  }
   /** Set the {@link FsPermission} of this {@link INode} */
   protected void setPermission(FsPermission permission) {
     updatePermissionStatus(PermissionStatusFormat.MODE, permission.toShort());
@@ -155,6 +158,14 @@ abstract class INode implements Comparable<byte[]> {
     return bytes2String(name);
   }
 
+  /**
+   * Get local file name
+   * @return local file name
+   */
+  byte[] getLocalNameBytes() {
+    return name;
+  }
+
   /**
    * Set local file name
    */
@@ -565,6 +576,9 @@ class INodeDirectory extends INode {
   List<INode> getChildren() {
     return children==null ? new ArrayList<INode>() : children;
   }
+  List<INode> getChildrenRaw() {
+    return children;
+  }
 
   /**
    * Collect all the blocks in all children of this INode.

+ 2 - 2
src/java/org/apache/hadoop/fs/permission/FsPermission.java

@@ -94,7 +94,7 @@ public class FsPermission implements Writable {
     groupaction = g;
     otheraction = o;
   }
-  private void fromShort(short n) {
+  public void fromShort(short n) {
     FsAction[] v = FsAction.values();
     set(v[(n >>> 6) & 7], v[(n >>> 3) & 7], v[n & 7]);
   }
@@ -194,4 +194,4 @@ public class FsPermission implements Writable {
     }
     return new FsPermission((short)n);
   }
-}
+}

+ 14 - 4
src/java/org/apache/hadoop/fs/permission/PermissionStatus.java

@@ -87,9 +87,7 @@ public class PermissionStatus implements Writable {
 
   /** {@inheritDoc} */
   public void write(DataOutput out) throws IOException {
-    Text.writeString(out, username);
-    Text.writeString(out, groupname);
-    permission.write(out);
+    write(out, username, groupname, permission);
   }
 
   /**
@@ -101,8 +99,20 @@ public class PermissionStatus implements Writable {
     return p;
   }
 
+  /**
+   * Serialize a {@link PermissionStatus} from its base components.
+   */
+  public static void write(DataOutput out,
+                           String username, 
+                           String groupname,
+                           FsPermission permission) throws IOException {
+    Text.writeString(out, username);
+    Text.writeString(out, groupname);
+    permission.write(out);
+  }
+
   /** {@inheritDoc} */
   public String toString() {
     return username + ":" + groupname + ":" + permission;
   }
-}
+}