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/06/19 00:44:10 UTC

svn commit: r548517 - in /lucene/hadoop/trunk: CHANGES.txt src/java/org/apache/hadoop/dfs/FSNamesystem.java src/java/org/apache/hadoop/dfs/PendingCreates.java

Author: cutting
Date: Mon Jun 18 15:44:09 2007
New Revision: 548517

URL: http://svn.apache.org/viewvc?view=rev&rev=548517
Log:
Fix HDFS block id generation to check pending blocks for duplicates.  Contributed by Dhruba.

Added:
    lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/PendingCreates.java
Modified:
    lucene/hadoop/trunk/CHANGES.txt
    lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSNamesystem.java

Modified: lucene/hadoop/trunk/CHANGES.txt
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/CHANGES.txt?view=diff&rev=548517&r1=548516&r2=548517
==============================================================================
--- lucene/hadoop/trunk/CHANGES.txt (original)
+++ lucene/hadoop/trunk/CHANGES.txt Mon Jun 18 15:44:09 2007
@@ -176,6 +176,9 @@
  54. HADOOP-1442.  Fix handling of zero-length input splits.
      (Senthil Subramanian via cutting)
 
+ 55. HADOOP-1444.  Fix HDFS block id generation to check pending
+     blocks for duplicates. (Dhruba Borthakur via cutting)
+
 
 Release 0.13.0 - 2007-06-08
 

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?view=diff&rev=548517&r1=548516&r2=548517
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSNamesystem.java (original)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSNamesystem.java Mon Jun 18 15:44:09 2007
@@ -103,16 +103,8 @@
   //
   // Keeps track of files that are being created, plus the
   // blocks that make them up.
-  // Mapping: fileName -> FileUnderConstruction
   //
-  Map<UTF8, FileUnderConstruction> pendingCreates = 
-    new TreeMap<UTF8, FileUnderConstruction>();
-
-  //
-  // Keeps track of the blocks that are part of those pending creates
-  // Set of: Block
-  //
-  Collection<Block> pendingCreateBlocks = new TreeSet<Block>();
+  PendingCreates pendingCreates = new PendingCreates();
 
   //
   // Stats on overall usage
@@ -833,23 +825,13 @@
     //
     NameNode.stateChangeLog.debug("BLOCK* NameSystem.abandonBlock: "
                                   +b.getBlockName()+"of file "+src);
-    FileUnderConstruction pendingFile = pendingCreates.get(src);
-    if (pendingFile != null) {
-      Collection<Block> pendingVector = pendingFile.getBlocks();
-      for (Iterator<Block> it = pendingVector.iterator(); it.hasNext();) {
-        Block cur = it.next();
-        if (cur.compareTo(b) == 0) {
-          pendingCreateBlocks.remove(cur);
-          it.remove();
-          NameNode.stateChangeLog.debug(
-                                        "BLOCK* NameSystem.abandonBlock: "
-                                        +b.getBlockName()
-                                        +" is removed from pendingCreateBlock and pendingCreates");
-          return true;
-        }
-      }
+    boolean status = pendingCreates.removeBlock(src, b);
+    if (status) {
+      NameNode.stateChangeLog.debug("BLOCK* NameSystem.abandonBlock: "
+                                    + b.getBlockName()
+                                    + " is removed from pendingCreates");
     }
-    return false;
+    return status;
   }
 
   /**
@@ -937,9 +919,6 @@
     NameNode.stateChangeLog.debug(
                                   "DIR* NameSystem.completeFile: " + src
                                   + " is removed from pendingCreates");
-    for (int i = 0; i < nrBlocks; i++) {
-      pendingCreateBlocks.remove(pendingBlocks[i]);
-    }
 
     synchronized (leases) {
       Lease lease = leases.get(holder);
@@ -985,9 +964,7 @@
     do {
       b = new Block(FSNamesystem.randBlockId.nextLong(), 0);
     } while (isValidBlock(b));
-    FileUnderConstruction v = pendingCreates.get(src);
-    v.getBlocks().add(b);
-    pendingCreateBlocks.add(b);
+    pendingCreates.addBlock(src, b);
     NameNode.stateChangeLog.info("BLOCK* NameSystem.allocateBlock: "
                                  +src+ ". "+b.getBlockName()+
                                  " is created and added to pendingCreates and pendingCreateBlocks");      
@@ -1403,19 +1380,16 @@
    * @param holder The datanode that was creating the file
    */
   private void internalReleaseCreate(UTF8 src, UTF8 holder) {
-    FileUnderConstruction v = pendingCreates.remove(src);
-    if (v != null) {
-      NameNode.stateChangeLog.debug(
-                                    "DIR* NameSystem.internalReleaseCreate: " + src
+    boolean status =  pendingCreates.remove(src);
+    if (status) {
+      NameNode.stateChangeLog.debug("DIR* NameSystem.internalReleaseCreate: " 
+                                    + src
                                     + " is removed from pendingCreates for "
                                     + holder + " (failure)");
-      for (Iterator<Block> it2 = v.getBlocks().iterator(); it2.hasNext();) {
-        Block b = it2.next();
-        pendingCreateBlocks.remove(b);
-      }
     } else {
       NameNode.stateChangeLog.warn("DIR* NameSystem.internalReleaseCreate: "
-                                   + "attempt to release a create lock on "+ src.toString()
+                                   + "attempt to release a create lock on "
+                                   + src.toString()
                                    + " that was not in pedingCreates");
     }
   }
@@ -2076,7 +2050,7 @@
       // they are added to recentInvalidateSets and will be sent out
       // thorugh succeeding heartbeat responses.
       //
-      if (!isValidBlock(b) && !pendingCreateBlocks.contains(b)) {
+      if (!isValidBlock(b)) {
         if (obsolete.size() > FSConstants.BLOCK_INVALIDATE_CHUNK) {
           addToInvalidates(b, node);
         } else {
@@ -3361,7 +3335,8 @@
   /**
    * Returns whether the given block is one pointed-to by a file.
    */
-  public boolean isValidBlock(Block b) {
-    return blocksMap.getINode(b) != null;
+  private boolean isValidBlock(Block b) {
+    return (blocksMap.getINode(b) != null ||
+            pendingCreates.contains(b));
   }
 }

Added: lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/PendingCreates.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/PendingCreates.java?view=auto&rev=548517
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/PendingCreates.java (added)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/PendingCreates.java Mon Jun 18 15:44:09 2007
@@ -0,0 +1,119 @@
+/**
+ * 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 org.apache.commons.logging.*;
+import org.apache.hadoop.io.UTF8;
+import java.io.*;
+import java.util.*;
+
+/***************************************************
+ * PendingCreates does the bookkeeping of all
+ * blocks that are in transit.
+ *
+ * It does the following:
+ * 1)  keep a map of pending-file to its blocks.
+ *     Mapping: fileName -> FileUnderConstruction
+ * 2)  a global set of all blocks that are part of all pending files.
+ *
+ * @author Dhruba Borthakur
+ ***************************************************/
+class PendingCreates {
+  private Map<UTF8, FileUnderConstruction> pendingCreates =
+                         new TreeMap<UTF8, FileUnderConstruction>();
+
+  //
+  // Keeps track of the blocks that are part of files that are being
+  // created. 
+  //
+  private Collection<Block> pendingCreateBlocks = new TreeSet<Block>();
+
+
+  //
+  // returns a file if it is being created. Otherwise returns null.
+  //
+  FileUnderConstruction get(UTF8 filename) {
+    return pendingCreates.get(filename);
+  }
+
+  //
+  // inserts a filename into pendingCreates. throws exception if it 
+  // already exists
+  //
+  void put(UTF8 src, FileUnderConstruction file) throws IOException {
+    FileUnderConstruction oldfile = pendingCreates.put(src, file);
+    if (oldfile != null && oldfile != file) {
+      throw new IOException("Duplicate entry " + src +
+                            " in pendingCreates.");
+    }
+  }
+
+  //
+  // The specified file is no longer pending.
+  //
+  boolean remove(UTF8 file) {
+    FileUnderConstruction v = pendingCreates.remove(file);
+    if (v != null) {
+      for (Iterator<Block> it2 = v.getBlocks().iterator(); it2.hasNext(); ) {
+        Block b = it2.next();
+        pendingCreateBlocks.remove(b);
+      }
+      return true;
+    }
+    return false;
+  }
+
+  //
+  // Make this block part of this file. This block
+  // should not already exists in here.
+  //
+  boolean addBlock(UTF8 file, Block b) {
+    FileUnderConstruction v =  pendingCreates.get(file);
+    assert !pendingCreateBlocks.contains(b);
+    v.getBlocks().add(b);
+    pendingCreateBlocks.add(b);
+    return true;
+  }
+
+  //
+  // Remove this block from a file.
+  //
+  boolean removeBlock(UTF8 file, Block b) {
+    FileUnderConstruction v =  pendingCreates.get(file);
+    if (v != null) {
+      Collection<Block> pendingVector = v.getBlocks();
+      for (Iterator<Block> it = pendingVector.iterator(); it.hasNext(); ) {
+        Block cur = it.next();
+        if (cur.compareTo(b) == 0) {
+          pendingCreateBlocks.remove(b);
+          it.remove();
+          return true;
+        }
+      }
+    }
+    return false;
+  }
+
+  //
+  // Returns true if this block is is pendingCreates
+  //
+  boolean contains(Block b) {
+    return pendingCreateBlocks.contains(b);
+  }
+}
+