You are viewing a plain text version of this content. The canonical link for it is here.
Posted to common-commits@hadoop.apache.org by cu...@apache.org on 2007/08/22 00:08:11 UTC

svn commit: r568303 - in /lucene/hadoop/trunk: ./ src/java/org/apache/hadoop/dfs/ src/test/org/apache/hadoop/dfs/

Author: cutting
Date: Tue Aug 21 15:08:08 2007
New Revision: 568303

URL: http://svn.apache.org/viewvc?rev=568303&view=rev
Log:
HADOOP-1743.  Change DFS INode from a nested class to a standalone class with specialized subclasses for directories and files.  Contributed by Konstantin.

Added:
    lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/INode.java
Modified:
    lucene/hadoop/trunk/CHANGES.txt
    lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/BlocksMap.java
    lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/DFSFileInfo.java
    lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSDirectory.java
    lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSEditLog.java
    lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSImage.java
    lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSNamesystem.java
    lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/LocatedBlocks.java
    lucene/hadoop/trunk/src/test/org/apache/hadoop/dfs/TestModTime.java

Modified: lucene/hadoop/trunk/CHANGES.txt
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/CHANGES.txt?rev=568303&r1=568302&r2=568303&view=diff
==============================================================================
--- lucene/hadoop/trunk/CHANGES.txt (original)
+++ lucene/hadoop/trunk/CHANGES.txt Tue Aug 21 15:08:08 2007
@@ -68,6 +68,10 @@
     HADOOP-1703.  DFS-internal code cleanups, removing several uses of
     the obsolete UTF8.  (Christophe Taton via cutting)
 
+    HADOOP-1743.  Change DFS INode from a nested class to standalone
+    class, with specialized subclasses for directories and files, to
+    save memory on the namenode.  (Konstantin Shvachko via cutting)
+
 
 Release 0.14.0 - 2007-08-17
 

Modified: lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/BlocksMap.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/BlocksMap.java?rev=568303&r1=568302&r2=568303&view=diff
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/BlocksMap.java (original)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/BlocksMap.java Tue Aug 21 15:08:08 2007
@@ -30,7 +30,7 @@
    * Internal class for block metadata.
    */
   static class BlockInfo {
-    private FSDirectory.INode              inode;
+    private INodeFile          inode;
       
     /** nodes could contain some null entries at the end, so 
      *  nodes.legth >= number of datanodes. 
@@ -73,12 +73,12 @@
     return info;
   }
       
-  public FSDirectory.INode getINode(Block b) {
+  public INodeFile getINode(Block b) {
     BlockInfo info = map.get(b);
     return (info != null) ? info.inode : null;
   }
           
-  public void addINode(Block b, FSDirectory.INode iNode) {
+  public void addINode(Block b, INodeFile iNode) {
     BlockInfo info = checkBlockInfo(b);
     info.inode = iNode;
   }

Modified: lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/DFSFileInfo.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/DFSFileInfo.java?rev=568303&r1=568302&r2=568303&view=diff
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/DFSFileInfo.java (original)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/DFSFileInfo.java Tue Aug 21 15:08:08 2007
@@ -46,11 +46,13 @@
   /**
    * Create DFSFileInfo by file INode 
    */
-  public DFSFileInfo(FSDirectory.INode node) {
+  public DFSFileInfo(String path, INode node) {
     // XXX This should probably let length == 0 for directories
-    super(node.isDir() ? node.computeContentsLength() : node.computeFileLength(),
-          node.isDir(), node.getReplication(), node.getBlockSize(),
-          node.getModificationTime(), new Path(node.getAbsoluteName()));
+    super(node.computeContentsLength(),
+          node.isDirectory(), 
+          node.isDirectory() ? 0 : ((INodeFile)node).getReplication(), 
+          node.isDirectory() ? 0 : ((INodeFile)node).getBlockSize(),
+          node.getModificationTime(), new Path(path));
   }
 
   /**

Modified: lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSDirectory.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSDirectory.java?rev=568303&r1=568302&r2=568303&view=diff
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSDirectory.java (original)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSDirectory.java Tue Aug 21 15:08:08 2007
@@ -19,8 +19,8 @@
 
 import java.io.*;
 import java.util.*;
-import org.apache.hadoop.conf.Configuration;
 
+import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.fs.Path;
 import org.apache.hadoop.metrics.MetricsRecord;
 import org.apache.hadoop.metrics.MetricsUtil;
@@ -37,389 +37,8 @@
  *************************************************/
 class FSDirectory implements FSConstants {
 
-  /******************************************************
-   * We keep an in-memory representation of the file/block
-   * hierarchy.
-   * 
-   * TODO: Factor out INode to a standalone class.
-   ******************************************************/
-  static class INode {
-
-    private String name;
-    private INode parent;
-    private List<INode> children = null;
-    private Block blocks[] = null;
-    private short blockReplication;
-    private long modificationTime;
-    private static final int DEFAULT_FILES_PER_DIRECTORY = 5;
-
-    /**
-     */
-    INode(String name) {
-      this.name = name;
-      this.parent = null;
-      this.blocks = null;
-      this.blockReplication = 0;
-      this.modificationTime = 0;
-    }
-
-    INode(String name, long modifictionTime) {
-      this.name = name;
-      this.parent = null;
-      this.blocks = null;
-      this.blockReplication = 0;
-      this.modificationTime = modifictionTime;
-    }
-
-    /**
-     */
-    INode(String name, Block blocks[], short replication,
-          long modificationTime) {
-      this.name = name;
-      this.parent = null;
-      this.blocks = blocks;
-      this.blockReplication = replication;
-      this.modificationTime = modificationTime;
-    }
-
-    /**
-     * Check whether it's a directory
-     */
-    synchronized public boolean isDir() {
-      return (blocks == null);
-    }
-        
-    /**
-     * Get block replication for the file 
-     * @return block replication
-     */
-    public short getReplication() {
-      return this.blockReplication;
-    }
-        
-    /**
-     * Get local file name
-     * @return local file name
-     */
-    String getLocalName() {
-      return name;
-    }
-
-    /**
-     * Set local file name
-     */
-    void setLocalName(String name) {
-      this.name = name;
-    }
-
-    /**
-     * Get the full absolute path name of this file (recursively computed).
-     * 
-     * @return the string representation of the absolute path of this file
-     */
-    String getAbsoluteName() {
-      if (this.parent == null) {    
-        return Path.SEPARATOR;       // root directory is "/"
-      }
-      return internalGetAbsolutePathName().toString();
-    }
-
-    /**
-     * Recursive computation of the absolute path name of this INode using a
-     * StringBuffer. This relies on the root INode name being "".
-     * 
-     * @return the StringBuffer containing the absolute path name.
-     */
-    private StringBuffer internalGetAbsolutePathName() {
-      if (parent == null) {
-        return new StringBuffer(name);
-      } else {
-        return parent.internalGetAbsolutePathName().append(
-            Path.SEPARATOR_CHAR).append(name);
-      }
-    }
-
-    /**
-     * Get file blocks 
-     * @return file blocks
-     */
-    Block[] getBlocks() {
-      return this.blocks;
-    }
-        
-    /**
-     * Get parent directory 
-     * @return parent INode
-     */
-    INode getParent() {
-      return this.parent;
-    }
-
-    /**
-     * Get last modification time of inode.
-     * @return access time
-     */
-    long getModificationTime() {
-      return this.modificationTime;
-    }
-
-    /**
-     * Set last modification time of inode.
-     */
-    void setModificationTime(long modtime) {
-      assert isDir();
-      if (this.modificationTime <= modtime) {
-        this.modificationTime = modtime;
-      }
-    }
-
-    /**
-     * Set file blocks 
-     * @return file blocks
-     */
-    void setBlocks(Block[] blockList) {
-      this.blocks = blockList;
-    }
-
-    /**
-     * Get children iterator
-     * @return Iterator of children
-     */
-    Iterator<INode> getChildIterator() {
-      return (children != null) ?  children.iterator() : null;
-      // instead of null, we could return a static empty iterator.
-    }
-        
-    private void addChild(String name, INode node) {
-      if (children == null) {
-        children = new ArrayList<INode>(DEFAULT_FILES_PER_DIRECTORY);
-      }
-      int low = Collections.binarySearch(children, node, comp);
-      assert low < 0;
-      children.add(-low - 1, node);
-    }
-
-    private void removeChild(INode node) {
-      assert children != null;
-      int low = Collections.binarySearch(children, node, comp);
-      if (low >= 0) {
-        children.remove(low);
-      }
-    }
-
-    /**
-     * This is the external interface
-     */
-    INode getNode(String target) {
-      if (target == null || 
-          !target.startsWith("/") || target.length() == 0) {
-        return null;
-      } else if (parent == null && "/".equals(target)) {
-        return this;
-      } else {
-        Vector<String> components = new Vector<String>();
-        int start = 0;
-        int slashid = 0;
-        while (start < target.length() && (slashid = target.indexOf('/', start)) >= 0) {
-          components.add(target.substring(start, slashid));
-          start = slashid + 1;
-        }
-        if (start < target.length()) {
-          components.add(target.substring(start));
-        }
-        return getNode(components, 0);
-      }
-    }
-
-    /**
-     */
-    INode getNode(Vector<String> components, int index) {
-      if (!name.equals(components.elementAt(index))) {
-        return null;
-      }
-      if (index == components.size()-1) {
-        return this;
-      }
-
-      // Check with children
-      INode child = this.getChild(components.elementAt(index+1));
-      if (child == null) {
-        return null;
-      } else {
-        return child.getNode(components, index+1);
-      }
-    }
-        
-    INode getChild(String name) {
-      return getChild(new INode(name));
-    }
-
-    private INode getChild(INode node) {
-      if (children == null) {
-        return null;
-      }
-      int low = Collections.binarySearch(children, node, comp);
-      if (low >= 0) {
-        return children.get(low);
-      }
-      return null;
-    }
-
-    //
-    // Prints out the children along with their hash codes.
-    // Useful for debugging.
-    //
-    private void printChildTree(List<INode> children) {
-      System.out.print("Names: ");
-      for (int i = 0; i < children.size(); i++) {
-        INode inode = children.get(i);
-        System.out.print(" " + inode.name);
-      }
-      System.out.println("");
-      System.out.print("Hashcodes: ");
-      for (int i = 0; i < children.size(); i++) {
-        INode inode = children.get(i);
-        System.out.print(" " + inode.name.hashCode());
-      }
-      System.out.println("");
-    }
-
-    /**
-     * Add new INode to the file tree.
-     * Find the parent and insert 
-     * 
-     * @param path file path
-     * @param newNode INode to be added
-     * @return null if the node already exists; inserted INode, otherwise
-     * @throws FileNotFoundException 
-     */
-    INode addNode(String path, INode newNode) throws FileNotFoundException {
-      File target = new File(path);
-      // find parent
-      Path parent = new Path(path).getParent();
-      if (parent == null) { // add root
-        return null;
-      }
-      INode parentNode = getNode(parent.toString());
-      if (parentNode == null) {
-        throw new FileNotFoundException(
-                                        "Parent path does not exist: "+path);
-      }
-      if (!parentNode.isDir()) {
-        throw new FileNotFoundException(
-                                        "Parent path is not a directory: "+path);
-      }
-      // check whether the parent already has a node with that name
-      String name =  target.getName();
-      newNode.setLocalName(name);
-      if (parentNode.getChild(newNode) != null) {
-        return null;
-      }
-      // insert into the parent children list
-      parentNode.addChild(name, newNode);
-      newNode.parent = parentNode;
-      return newNode;
-    }
-
-    /**
-     */
-    boolean removeNode() {
-      if (parent == null) {
-        return false;
-      } else {
-        parent.removeChild(this);
-        return true;
-      }
-    }
-
-    /**
-     * Collect all the blocks at this INode and all its children.
-     * This operation is performed after a node is removed from the tree,
-     * and we want to GC all the blocks at this node and below.
-     */
-    void collectSubtreeBlocks(FSDirectory fsDir, Vector<Block> v) {
-      if (blocks != null) {
-        for (int i = 0; i < blocks.length; i++) {
-          v.add(blocks[i]);
-        }
-      }
-      fsDir.incrDeletedFileCount();
-      for (Iterator<INode> it = getChildIterator(); it != null &&
-             it.hasNext();) {
-        it.next().collectSubtreeBlocks(fsDir, v);
-      }
-    }
-
-    /**
-     */
-    int numItemsInTree() {
-      int total = 0;
-      for (Iterator<INode> it = getChildIterator(); it != null && 
-             it.hasNext();) {
-        total += it.next().numItemsInTree();
-      }
-      return total + 1;
-    }
-
-    long computeFileLength() {
-      long total = 0;
-      if (blocks != null) {
-        for (int i = 0; i < blocks.length; i++) {
-          total += blocks[i].getNumBytes();
-        }
-      }
-      return total;
-    }
-
-    /**
-     */
-    long computeContentsLength() {
-      long total = computeFileLength();
-      for (Iterator<INode> it = getChildIterator(); it != null && 
-             it.hasNext();) {
-        total += it.next().computeContentsLength();
-      }
-      return total;
-    }
-
-    /**
-     * Get the block size of the first block
-     * @return the number of bytes
-     */
-    public long getBlockSize() {
-      if (blocks == null || blocks.length == 0) {
-        return 0;
-      } else {
-        return blocks[0].getNumBytes();
-      }
-    }
-        
-    /**
-     */
-    void listContents(Vector<INode> v) {
-      if (parent != null && blocks != null) {
-        v.add(this);
-      }
-
-      for (Iterator<INode> it = getChildIterator(); it != null && 
-             it.hasNext();) {
-        v.add(it.next());
-      }
-    }
-
-    //
-    // Use the name of the INode to implement a natural order 
-    //
-    static private final Comparator<INode> comp =
-      new Comparator<INode>() {
-        public int compare(INode a, INode b) {
-          return a.getLocalName().compareTo(b.getLocalName());
-        }
-      };
-  }
-
   FSNamesystem namesystem = null;
-  INode rootDir = new INode("");
+  INodeDirectory rootDir = new INodeDirectory(INodeDirectory.ROOT_NAME);
   FSImage fsImage;  
   boolean ready = false;
   // Metrics record
@@ -464,8 +83,8 @@
     }
   }
 
-  private void incrDeletedFileCount() {
-    directoryMetrics.incrMetric("files_deleted", 1);
+  private void incrDeletedFileCount(int count) {
+    directoryMetrics.incrMetric("files_deleted", count);
     directoryMetrics.update();
   }
     
@@ -503,16 +122,15 @@
     if (!mkdirs(new Path(path).getParent().toString(), modTime)) {
       return false;
     }
-    INode newNode = new INode(new File(path).getName(), blocks, 
-                              replication, modTime);
-    if (!unprotectedAddFile(path, newNode, modTime)) {
+    INodeFile newNode = (INodeFile)unprotectedAddFile(path, blocks, replication, modTime);
+    if (newNode == null) {
       NameNode.stateChangeLog.info("DIR* FSDirectory.addFile: "
                                    +"failed to add "+path+" with "
                                    +blocks.length+" blocks to the file system");
       return false;
     }
     // add create file record to log
-    fsImage.getEditLog().logCreateFile(newNode);
+    fsImage.getEditLog().logCreateFile(path, newNode);
     NameNode.stateChangeLog.debug("DIR* FSDirectory.addFile: "
                                   +path+" with "+blocks.length+" blocks is added to the file system");
     return true;
@@ -520,34 +138,31 @@
     
   /**
    */
-  private boolean unprotectedAddFile(String path, INode newNode, long parentModTime) {
+  INode unprotectedAddFile( String path, 
+                            Block[] blocks, 
+                            short replication,
+                            long modificationTime) {
+    INode newNode;
+    if (blocks == null)
+      newNode = new INodeDirectory(modificationTime);
+    else
+      newNode = new INodeFile(blocks, replication, modificationTime);
     synchronized (rootDir) {
       try {
-        if (rootDir.addNode(path, newNode) != null) {
-          int nrBlocks = (newNode.blocks == null) ? 0 : newNode.blocks.length;
+        newNode = rootDir.addNode(path, newNode);
+        if(newNode != null) {
+          int nrBlocks = (blocks == null) ? 0 : blocks.length;
           // Add file->block mapping
           for (int i = 0; i < nrBlocks; i++) {
-            namesystem.blocksMap.addINode(newNode.blocks[i], newNode);
+            namesystem.blocksMap.addINode(blocks[i], (INodeFile)newNode);
           }
-          // update modification time of parent directory
-          newNode.getParent().setModificationTime(parentModTime);
-          return true;
-        } else {
-          return false;
         }
       } catch (FileNotFoundException e) {
-        return false;
+        return null;
       }
+      return newNode;
     }
   }
-    
-  boolean unprotectedAddFile(String path, Block[] blocks, short replication,
-                             long modificationTime) {
-    return unprotectedAddFile(path,  
-                              new INode(path, blocks, replication,
-                                        modificationTime),
-                              modificationTime);
-  }
 
   /**
    * Add blocks to the file.
@@ -556,7 +171,7 @@
     waitForReady();
 
     synchronized (rootDir) {
-      INode fileNode = rootDir.getNode(path);
+      INodeFile fileNode = this.getFileINode(path);
       if (fileNode == null) {
         throw new IOException("Unknown file: " + path);
       }
@@ -578,7 +193,7 @@
       fsImage.getEditLog().logDelete(path, fileNode.getModificationTime());
 
       // re-add create file record to log
-      fsImage.getEditLog().logCreateFile(fileNode);
+      fsImage.getEditLog().logCreateFile(path, fileNode);
       NameNode.stateChangeLog.debug("DIR* FSDirectory.addFile: "
                                     +path+" with "+blocks.length
                                     +" blocks is added to the file system");
@@ -593,7 +208,7 @@
     NameNode.stateChangeLog.debug("DIR* FSDirectory.renameTo: "
                                   +src+" to "+dst);
     waitForReady();
-    long now = namesystem.now();
+    long now = FSNamesystem.now();
     if (!unprotectedRenameTo(src, dst, now))
       return false;
     fsImage.getEditLog().logRename(src, dst, now);
@@ -618,8 +233,8 @@
                                      +"failed to rename "+src+" to "+dst+ " because destination exists");
         return false;
       }
-      INode oldParent = renamedNode.getParent();
-      renamedNode.removeNode();
+      INodeDirectory oldParent = renamedNode.getParent();
+      oldParent.removeChild(renamedNode);
             
       // the renamed node can be reused now
       try {
@@ -673,14 +288,15 @@
     oldReplication[0] = -1;
     Block[] fileBlocks = null;
     synchronized(rootDir) {
-      INode fileNode = rootDir.getNode(src);
-      if (fileNode == null)
+      INode inode = rootDir.getNode(src);
+      if (inode == null)
         return null;
-      if (fileNode.isDir())
+      if (inode.isDirectory())
         return null;
-      oldReplication[0] = fileNode.blockReplication;
-      fileNode.blockReplication = replication;
-      fileBlocks = fileNode.blocks;
+      INodeFile fileNode = (INodeFile)inode;
+      oldReplication[0] = fileNode.getReplication();
+      fileNode.setReplication(replication);
+      fileBlocks = fileNode.getBlocks();
     }
     return fileBlocks;
   }
@@ -697,11 +313,11 @@
       if (fileNode == null) {
         throw new IOException("Unknown file: " + filename);
       }
-      if (fileNode.isDir()) {
+      if (fileNode.isDirectory()) {
         throw new IOException("Getting block size of a directory: " + 
                               filename);
       }
-      return fileNode.getBlockSize();
+      return ((INodeFile)fileNode).getBlockSize();
     }
   }
     
@@ -712,7 +328,7 @@
     NameNode.stateChangeLog.debug("DIR* FSDirectory.delete: "
                                   +src);
     waitForReady();
-    long now = namesystem.now();
+    long now = FSNamesystem.now();
     Block[] blocks = unprotectedDelete(src, now); 
     if (blocks != null)
       fsImage.getEditLog().logDelete(src, now);
@@ -741,8 +357,9 @@
           NameNode.stateChangeLog.debug("DIR* FSDirectory.unprotectedDelete: "
                                         +src+" is removed");
           targetNode.getParent().setModificationTime(modificationTime);
-          Vector<Block> v = new Vector<Block>();
-          targetNode.collectSubtreeBlocks(this, v);
+          ArrayList<Block> v = new ArrayList<Block>();
+          int filesRemoved = targetNode.collectSubtreeBlocks(v);
+          incrDeletedFileCount(filesRemoved);
           for (Block b : v) {
             namesystem.blocksMap.removeINode(b);
           }
@@ -763,19 +380,21 @@
 
     synchronized (rootDir) {
       INode targetNode = rootDir.getNode(srcs);
-      if (targetNode == null) {
+      if (targetNode == null)
         return null;
-      } else {
-        Vector<INode> contents = new Vector<INode>();
-        targetNode.listContents(contents);
-
-        DFSFileInfo listing[] = new DFSFileInfo[contents.size()];
-        int i = 0;
-        for (Iterator<INode> it = contents.iterator(); it.hasNext(); i++) {
-          listing[i] = new DFSFileInfo(it.next());
-        }
-        return listing;
+      if (!targetNode.isDirectory()) {
+        return new DFSFileInfo[]{new DFSFileInfo(srcs, targetNode)};
       }
+      List<INode> contents = ((INodeDirectory)targetNode).getChildren();
+      DFSFileInfo listing[] = new DFSFileInfo[contents.size()];
+      if(! srcs.endsWith(Path.SEPARATOR))
+        srcs += Path.SEPARATOR;
+      int i = 0;
+      for (INode cur : contents) {
+        listing[i] = new DFSFileInfo(srcs+cur.getLocalName(), cur);
+        i++;
+      }
+      return listing;
     }
   }
 
@@ -792,7 +411,7 @@
         throw new IOException("File does not exist");
       }
       else {
-        return new DFSFileInfo(targetNode);
+        return new DFSFileInfo(srcs, targetNode);
       }
     }
   }
@@ -804,21 +423,24 @@
     waitForReady();
     synchronized (rootDir) {
       INode targetNode = rootDir.getNode(src);
-      if (targetNode == null) {
+      if (targetNode == null)
         return null;
-      } else {
-        return targetNode.blocks;
-      }
+      if(targetNode.isDirectory())
+        return null;
+      return ((INodeFile)targetNode).getBlocks();
     }
   }
 
   /**
    * Get {@link INode} associated with the file.
    */
-  INode getFileINode(String src) {
+  INodeFile getFileINode(String src) {
     waitForReady();
     synchronized (rootDir) {
-      return rootDir.getNode(src);
+      INode inode = rootDir.getNode(src);
+      if (inode == null || inode.isDirectory())
+        return null;
+      return (INodeFile)inode;
     }
   }
 
@@ -844,7 +466,7 @@
   public boolean isDir(String src) {
     synchronized (rootDir) {
       INode node = rootDir.getNode(normalizePath(src));
-      return node != null && node.isDir();
+      return node != null && node.isDirectory();
     }
   }
 
@@ -855,7 +477,7 @@
     src = normalizePath(src);
 
     // Use this to collect all the dirs we need to construct
-    Vector<String> v = new Vector<String>();
+    List<String> v = new ArrayList<String>();
 
     // The dir itself
     v.add(src);
@@ -871,13 +493,13 @@
     // the way
     int numElts = v.size();
     for (int i = numElts - 1; i >= 0; i--) {
-      String cur = v.elementAt(i);
+      String cur = v.get(i);
       try {
         INode inserted = unprotectedMkdir(cur, now);
         if (inserted != null) {
           NameNode.stateChangeLog.debug("DIR* FSDirectory.mkdirs: "
                                         +"created directory "+cur);
-          fsImage.getEditLog().logMkDir(inserted);
+          fsImage.getEditLog().logMkDir(cur, inserted);
         } else { // otherwise cur exists, verify that it is a directory
           if (!isDir(cur)) {
             NameNode.stateChangeLog.debug("DIR* FSDirectory.mkdirs: "
@@ -898,8 +520,7 @@
    */
   INode unprotectedMkdir(String src, long timestamp) throws FileNotFoundException {
     synchronized (rootDir) {
-      return rootDir.addNode(src, new INode(new File(src).getName(),
-                                            timestamp));
+      return rootDir.addNode(src, new INodeDirectory(timestamp));
     }
   }
 

Modified: lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSEditLog.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSEditLog.java?rev=568303&r1=568302&r2=568303&view=diff
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSEditLog.java (original)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSEditLog.java Tue Aug 21 15:08:08 2007
@@ -476,9 +476,9 @@
   /** 
    * Add create file record to edit log
    */
-  void logCreateFile(FSDirectory.INode newNode) {
+  void logCreateFile(String path, INodeFile newNode) {
     UTF8 nameReplicationPair[] = new UTF8[] { 
-      new UTF8(newNode.getAbsoluteName()), 
+      new UTF8(path), 
       FSEditLog.toLogReplication(newNode.getReplication()),
       FSEditLog.toLogTimeStamp(newNode.getModificationTime())};
     logEdit(OP_ADD,
@@ -489,9 +489,9 @@
   /** 
    * Add create directory record to edit log
    */
-  void logMkDir(FSDirectory.INode newNode) {
+  void logMkDir(String path, INode newNode) {
     UTF8 info[] = new UTF8[] {
-      new UTF8(newNode.getAbsoluteName()),
+      new UTF8(path),
       FSEditLog.toLogTimeStamp(newNode.getModificationTime())
     };
     logEdit(OP_MKDIR, new ArrayWritable(UTF8.class, info), null);

Modified: lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSImage.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSImage.java?rev=568303&r1=568302&r2=568303&view=diff
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSImage.java (original)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSImage.java Tue Aug 21 15:08:08 2007
@@ -38,7 +38,6 @@
 
 import org.apache.hadoop.dfs.FSConstants.StartupOption;
 import org.apache.hadoop.dfs.FSConstants.NodeType;
-import org.apache.hadoop.dfs.FSDirectory.INode;
 import org.apache.hadoop.io.UTF8;
 import org.apache.hadoop.io.WritableComparable;
 
@@ -638,7 +637,6 @@
     // Load in bits
     //
     boolean needToSave = true;
-    int imgVersion;
     DataInputStream in = new DataInputStream(
                                              new BufferedInputStream(
                                                                      new FileInputStream(curFile)));
@@ -648,7 +646,7 @@
        * it should not contain version and namespace fields
        */
       // read image version: first appeared in version -1
-      imgVersion = in.readInt();
+      int imgVersion = in.readInt();
       // read namespaceID: first appeared in version -2
       if (imgVersion <= -2)
         this.namespaceID = in.readInt();
@@ -806,26 +804,29 @@
    * Save file tree image starting from the given root.
    */
   private static void saveImage(String parentPrefix, 
-                                FSDirectory.INode root, 
+                                INode inode, 
                                 DataOutputStream out) throws IOException {
     String fullName = "";
-    if (root.getParent() != null) {
-      fullName = parentPrefix + "/" + root.getLocalName();
+    if (inode.getParent() != null) {
+      fullName = parentPrefix + "/" + inode.getLocalName();
       new UTF8(fullName).write(out);
-      out.writeShort(root.getReplication());
-      out.writeLong(root.getModificationTime());
-      if (root.isDir()) {
-        out.writeInt(0);
-      } else {
-        int nrBlocks = root.getBlocks().length;
-        out.writeInt(nrBlocks);
-        for (int i = 0; i < nrBlocks; i++)
-          root.getBlocks()[i].write(out);
+      if (!inode.isDirectory()) {  // write file inode
+        INodeFile fileINode = (INodeFile)inode;
+        out.writeShort(fileINode.getReplication());
+        out.writeLong(inode.getModificationTime());
+        Block[] blocks = fileINode.getBlocks();
+        out.writeInt(blocks.length);
+        for (Block blk : blocks)
+          blk.write(out);
+        return;
       }
+      // write directory inode
+      out.writeShort(0);  // replication
+      out.writeLong(inode.getModificationTime());
+      out.writeInt(0);    // # of blocks
     }
-    for(Iterator<INode> it = root.getChildIterator(); it != null &&
-          it.hasNext();) {
-      saveImage(fullName, it.next(), out);
+    for(INode child : ((INodeDirectory)inode).getChildren()) {
+      saveImage(fullName, child, out);
     }
   }
 

Modified: lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSNamesystem.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSNamesystem.java?rev=568303&r1=568302&r2=568303&view=diff
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSNamesystem.java (original)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSNamesystem.java Tue Aug 21 15:08:08 2007
@@ -19,7 +19,6 @@
 
 import org.apache.commons.logging.*;
 
-import org.apache.hadoop.io.*;
 import org.apache.hadoop.conf.*;
 import org.apache.hadoop.util.*;
 import org.apache.hadoop.mapred.StatusHttpServer;
@@ -406,12 +405,12 @@
     
   /* get replication factor of a block */
   private int getReplication(Block block) {
-    FSDirectory.INode fileINode = blocksMap.getINode(block);
+    INodeFile fileINode = blocksMap.getINode(block);
     if (fileINode == null) { // block does not belong to any file
       return 0;
-    } else {
-      return fileINode.getReplication();
     }
+    assert !fileINode.isDirectory() : "Block cannot belong to a directory.";
+    return fileINode.getReplication();
   }
 
   /* updates a block in under replication queue */
@@ -441,8 +440,8 @@
     BlockCrcInfo crcInfo = new BlockCrcInfo();
     crcInfo.status = BlockCrcInfo.STATUS_ERROR;
     
-    FSDirectory.INode fileINode = blocksMap.getINode(block);
-    if ( fileINode == null || fileINode.isDir() ) {
+    INodeFile fileINode = blocksMap.getINode(block);
+    if ( fileINode == null || fileINode.isDirectory() ) {
       // Most probably reason is that this block does not exist
       if (blocksMap.getStoredBlock(block) == null) {
         crcInfo.status = BlockCrcInfo.STATUS_UNKNOWN_BLOCK;
@@ -496,9 +495,9 @@
 
       //Find CRC file
       String crcName = "." + fileName + ".crc";
-      FSDirectory.INode crcINode = fileINode.getParent().getChild(crcName);
+      INodeFile crcINode = (INodeFile)fileINode.getParent().getChild(crcName);
 
-      if ( crcINode == null ) {
+      if (crcINode == null ) {
         // Should we log this?
         crcInfo.status = BlockCrcInfo.STATUS_NO_CRC_DATA;
         return crcInfo;
@@ -569,11 +568,11 @@
     return blocks;
   }
   
-  private synchronized LocatedBlocks getBlockLocations(FSDirectory.INode inode, 
+  private synchronized LocatedBlocks getBlockLocations(INodeFile inode, 
                                                        long offset, 
                                                        long length,
                                                        int nrBlocksToReturn) {
-    if(inode == null || inode.isDir()) {
+    if(inode == null) {
       return null;
     }
     Block[] blocks = inode.getBlocks();
@@ -2177,7 +2176,7 @@
    */
   synchronized Block addStoredBlock(Block block, DatanodeDescriptor node) {
         
-    FSDirectory.INode fileINode = blocksMap.getINode(block);
+    INodeFile fileINode = blocksMap.getINode(block);
     int replication = (fileINode != null) ?  fileINode.getReplication() : 
       defaultReplication;
     boolean added = blocksMap.addNode(block, node, replication);
@@ -2388,7 +2387,7 @@
     // necessary.  In that case, put block on a possibly-will-
     // be-replicated list.
     //
-    FSDirectory.INode fileINode = blocksMap.getINode(block);
+    INode fileINode = blocksMap.getINode(block);
     if (fileINode != null) {
       updateNeededReplications(block, -1, 0);
     }
@@ -2712,7 +2711,7 @@
     boolean status = false;
     for (int i = 0; i < decommissionBlocks.length; i++) {
       Block block = decommissionBlocks[i];
-      FSDirectory.INode fileINode = blocksMap.getINode(block);
+      INode fileINode = blocksMap.getINode(block);
 
       if (fileINode != null) {
         NumberReplicas num = countNodes(block);
@@ -2797,7 +2796,7 @@
           }
           Block block = it.next();
           long blockSize = block.getNumBytes();
-          FSDirectory.INode fileINode = blocksMap.getINode(block);
+          INodeFile fileINode = blocksMap.getINode(block);
           if (fileINode == null) { // block does not belong to any file
             it.remove();
           } else {

Added: lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/INode.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/INode.java?rev=568303&view=auto
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/INode.java (added)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/INode.java Tue Aug 21 15:08:08 2007
@@ -0,0 +1,409 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.hadoop.dfs;
+
+import java.io.FileNotFoundException;
+import java.util.ArrayList;
+import java.util.Collections;
+import java.util.List;
+
+import org.apache.hadoop.fs.Path;
+
+/**
+ * We keep an in-memory representation of the file/block hierarchy.
+ * This is a base INode class containing common fields for file and 
+ * directory inodes.
+ */
+abstract class INode implements Comparable<String> {
+  protected String name;
+  protected INodeDirectory parent;
+  protected long modificationTime;
+
+  protected INode(String name) {
+    this(0L);
+    this.name = name;
+  }
+
+  INode(long mTime) {
+    this.modificationTime = mTime;
+  }
+
+  /**
+   * Check whether it's a directory
+   */
+  abstract boolean isDirectory();
+  abstract int collectSubtreeBlocks(List<Block> v);
+  abstract long computeContentsLength();
+
+  /**
+   * Get local file name
+   * @return local file name
+   */
+  String getLocalName() {
+    return name;
+  }
+
+  /**
+   * Set local file name
+   */
+  void setLocalName(String name) {
+    this.name = name;
+  }
+
+  /**
+   * Get the full absolute path name of this file (recursively computed).
+   * 
+   * @return the string representation of the absolute path of this file
+   * 
+   * @deprecated this is used only in crc upgrade now, and should be removed 
+   * in order to be able to eliminate the parent field. 
+   */
+  String getAbsoluteName() {
+    if (this.parent == null) {
+      return Path.SEPARATOR;       // root directory is "/"
+    }
+    if (this.parent.parent == null) {
+      return Path.SEPARATOR + name;
+    }
+    return parent.getAbsoluteName() + Path.SEPARATOR + name;
+  }
+
+  /**
+   * Get parent directory 
+   * @return parent INode
+   */
+  INodeDirectory getParent() {
+    return this.parent;
+  }
+
+  /**
+   * Get last modification time of inode.
+   * @return access time
+   */
+  long getModificationTime() {
+    return this.modificationTime;
+  }
+
+  /**
+   * Set last modification time of inode.
+   */
+  void setModificationTime(long modtime) {
+    assert isDirectory();
+    if (this.modificationTime <= modtime) {
+      this.modificationTime = modtime;
+    }
+  }
+
+  static String[] getPathComponents(String path) {
+    if (path == null || !path.startsWith(Path.SEPARATOR)) {
+      return null;
+    }
+    if (Path.SEPARATOR.equals(path))  // root
+      return new String[]{""};
+    return path.split(Path.SEPARATOR, -1);
+  }
+
+  /**
+   */
+  boolean removeNode() {
+    if (parent == null) {
+      return false;
+    } else {
+      
+      parent.removeChild(this);
+      return true;
+    }
+  }
+
+  //
+  // Comparable interface
+  //
+  public int compareTo(String o) {
+    return getLocalName().compareTo(o);
+  }
+
+  public boolean equals(Object o) {
+    if (!(o instanceof INode)) {
+      return false;
+    }
+    return getLocalName().equals(((INode)o).getLocalName());
+  }
+    
+  public int hashCode() {
+    return getLocalName().hashCode();
+  }
+}
+
+/**
+ * Directory INode class.
+ */
+class INodeDirectory extends INode {
+  protected static final int DEFAULT_FILES_PER_DIRECTORY = 5;
+  final static String ROOT_NAME = "";
+
+  private List<INode> children;
+
+  INodeDirectory(String name) {
+    super(name);
+    this.children = null;
+  }
+
+  INodeDirectory(long mTime) {
+    super(mTime);
+    this.children = null;
+  }
+
+  /**
+   * Check whether it's a directory
+   */
+  boolean isDirectory() {
+    return true;
+  }
+
+  void removeChild(INode node) {
+    assert children != null;
+    int low = Collections.binarySearch(children, node.getLocalName());
+    if (low >= 0) {
+      children.remove(low);
+    }
+  }
+
+  INode getChild(String name) {
+    if (children == null) {
+      return null;
+    }
+    int low = Collections.binarySearch(children, name);
+    if (low >= 0) {
+      return children.get(low);
+    }
+    return null;
+  }
+
+  /**
+   */
+  private INode getNode(String[] components) {
+    return getINode(components, components.length-1);
+  }
+
+  /**
+   * Find INode in the directory tree.
+   * 
+   * @param components array of path name components
+   * @param end the end component of the path
+   * @return found INode or null otherwise 
+   */
+  private INode getINode(String[] components, int end) {
+    assert getLocalName().equals(components[0]) :
+      "Incorrect name " + getLocalName() + " expected " + components[0];
+    if (end >= components.length)
+      end = components.length-1;
+    if (end < 0)
+      return null;
+    INode curNode = this;
+    for(int start = 0; start < end; start++) {
+      if(!curNode.isDirectory())  // file is not expected here
+        return null;        // because there is more components in the path
+      INodeDirectory parentDir = (INodeDirectory)curNode;
+      curNode = parentDir.getChild(components[start+1]);
+      if(curNode == null)  // not found
+        return null;
+    }
+    return curNode;
+  }
+
+  /**
+   * This is the external interface
+   */
+  INode getNode(String path) {
+    return getNode(getPathComponents(path));
+  }
+
+  /**
+   * Add a child inode to the directory.
+   * 
+   * @param node INode to insert
+   * @return  null if the child with this name already exists; 
+   *          inserted INode, otherwise
+   */
+  private <T extends INode> T addChild(T node) {
+    if (children == null) {
+      children = new ArrayList<INode>(DEFAULT_FILES_PER_DIRECTORY);
+    }
+    int low = Collections.binarySearch(children, node.getLocalName());
+    if(low >= 0)
+      return null;
+    node.parent = this;
+    children.add(-low - 1, node);
+    // update modification time of the parent directory
+    setModificationTime(node.getModificationTime());
+    return node;
+  }
+
+  /**
+   * Add new INode to the file tree.
+   * Find the parent and insert 
+   * 
+   * @param path file path
+   * @param newNode INode to be added
+   * @return null if the node already exists; inserted INode, otherwise
+   * @throws FileNotFoundException 
+   */
+  <T extends INode> T addNode(String path, T newNode) throws FileNotFoundException {
+    String[] pathComponents = getPathComponents(path);
+    assert pathComponents != null : "Incorrect path " + path;
+    int pathLen = pathComponents.length;
+    if (pathLen < 2)  // add root
+      return null;
+    INode node = getINode(pathComponents, pathLen-2);
+    if (node == null) {
+      throw new FileNotFoundException("Parent path does not exist: "+path);
+    }
+    if (!node.isDirectory()) {
+      throw new FileNotFoundException("Parent path is not a directory: "+path);
+    }
+    INodeDirectory parentNode = (INodeDirectory)node;
+    // insert into the parent children list
+    newNode.setLocalName(pathComponents[pathLen-1]);
+    return parentNode.addChild(newNode);
+  }
+
+  /**
+   */
+  int numItemsInTree() {
+    int total = 1;
+    if (children == null) {
+      return total;
+    }
+    for (INode child : children) {
+      if(!child.isDirectory())
+        total++;
+      else
+        total += ((INodeDirectory)child).numItemsInTree();
+    }
+    return total;
+  }
+
+  /**
+   */
+  long computeContentsLength() {
+    long total = 0;
+    if (children == null) {
+      return total;
+    }
+    for (INode child : children) {
+      total += child.computeContentsLength();
+    }
+    return total;
+  }
+
+  /**
+   */
+  List<INode> getChildren() {
+    return children==null ? new ArrayList<INode>() : children;
+  }
+
+  /**
+   * Collect all the blocks in all children of this INode.
+   * Count and return the number of files in the sub tree.
+   */
+  int collectSubtreeBlocks(List<Block> v) {
+    int total = 1;
+    if (children == null) {
+      return total;
+    }
+    for (INode child : children) {
+      total += child.collectSubtreeBlocks(v);
+    }
+    return total;
+  }
+}
+
+class INodeFile extends INode {
+  private Block blocks[] = null;
+  protected short blockReplication;
+
+  /**
+   */
+  INodeFile(Block blocks[], short replication, long modificationTime) {
+    super(modificationTime);
+    this.blocks = blocks;
+    this.blockReplication = replication;
+  }
+
+  boolean isDirectory() {
+    return false;
+  }
+
+  /**
+   * Get block replication for the file 
+   * @return block replication
+   */
+  short getReplication() {
+    return this.blockReplication;
+  }
+
+  void setReplication(short replication) {
+    this.blockReplication = replication;
+  }
+
+  /**
+   * Get file blocks 
+   * @return file blocks
+   */
+  Block[] getBlocks() {
+    return this.blocks;
+  }
+
+  /**
+   * Set file blocks 
+   */
+  void setBlocks(Block[] blockList) {
+    this.blocks = blockList;
+  }
+
+  /**
+   * Collect all the blocks in this INode.
+   * Return the number of files in the sub tree.
+   */
+  int collectSubtreeBlocks(List<Block> v) {
+    for (Block blk : blocks) {
+      v.add(blk);
+    }
+    return 1;
+  }
+
+  long computeContentsLength() {
+    long total = 0;
+    for (Block blk : blocks) {
+      total += blk.getNumBytes();
+    }
+    return total;
+  }
+
+  /**
+   * Get the block size of the first block
+   * @return the number of bytes
+   */
+  long getBlockSize() {
+    if (blocks == null || blocks.length == 0) {
+      return 0;
+    } else {
+      return blocks[0].getNumBytes();
+    }
+  }
+}

Modified: lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/LocatedBlocks.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/LocatedBlocks.java?rev=568303&r1=568302&r2=568303&view=diff
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/LocatedBlocks.java (original)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/LocatedBlocks.java Tue Aug 21 15:08:08 2007
@@ -41,8 +41,8 @@
     blocks = null;
   }
   
-  LocatedBlocks(FSDirectory.INode inode, List<LocatedBlock> blks) {
-    fileLength = inode.computeFileLength();
+  LocatedBlocks(INodeFile inode, List<LocatedBlock> blks) {
+    fileLength = inode.computeContentsLength();
     blocks = blks;
   }
   

Modified: lucene/hadoop/trunk/src/test/org/apache/hadoop/dfs/TestModTime.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/test/org/apache/hadoop/dfs/TestModTime.java?rev=568303&r1=568302&r2=568303&view=diff
==============================================================================
--- lucene/hadoop/trunk/src/test/org/apache/hadoop/dfs/TestModTime.java (original)
+++ lucene/hadoop/trunk/src/test/org/apache/hadoop/dfs/TestModTime.java Tue Aug 21 15:08:08 2007
@@ -178,4 +178,8 @@
       cluster.shutdown();
     }
   }
+
+  public static void main(String[] args) throws Exception {
+    new TestModTime().testModTime();
+  }
 }