浏览代码

HDFS-603. Add a new interface, Replica, which is going to replace the use of Block in datanode.

git-svn-id: https://svn.apache.org/repos/asf/hadoop/hdfs/branches/HDFS-265@813111 13f79535-47bb-0310-9956-ffa450edef68
Tsz-wo Sze 15 年之前
父节点
当前提交
567ebb43ba

+ 3 - 0
CHANGES.txt

@@ -33,6 +33,9 @@ Append branch (unreleased changes)
     HDFS-543. Break FSDatasetInterface#writToBlock() into writeToRemporary,
     writeToRBW, ad append. (hairong)
 
+    HDFS-603. Add a new interface, Replica, which is going to replace the use
+    of Block in datanode.  (szetszwo)
+
   BUG FIXES
 
     HDFS-547. TestHDFSFileSystemContract#testOutputStreamClosedTwice

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

@@ -1832,14 +1832,8 @@ public class FSDataset implements FSConstants, FSDatasetInterface {
     }
   }
 
-  /**
-   * Get reference to the replica meta info in the replicasMap. 
-   * To be called from methods that
-   * are synchronized on {@link FSDataset}
-   * @param blockId
-   * @return replica's meta information from the replicas map
-   */
-  ReplicaInfo getBlock(long blockId) {
+  @Override
+  public ReplicaInfo getReplica(long blockId) {
     assert(Thread.holdsLock(this));
     return volumeMap.get(blockId);
   }

+ 8 - 0
src/java/org/apache/hadoop/hdfs/server/datanode/FSDatasetInterface.java

@@ -94,6 +94,14 @@ public interface FSDatasetInterface extends FSDatasetMBean {
    */
   public long getLength(Block b) throws IOException;
 
+  /**
+   * Get reference to the replica meta info in the replicasMap. 
+   * To be called from methods that are synchronized on {@link FSDataset}
+   * @param blockId
+   * @return replica from the replicas map
+   */
+  public Replica getReplica(long blockId);
+
   /**
    * @return the generation stamp stored with the block.
    */

+ 14 - 4
src/java/org/apache/hadoop/hdfs/server/datanode/FinalizedReplica.java

@@ -18,7 +18,6 @@
 package org.apache.hadoop.hdfs.server.datanode;
 
 import java.io.File;
-import java.io.IOException;
 
 import org.apache.hadoop.hdfs.protocol.Block;
 import org.apache.hadoop.hdfs.server.common.HdfsConstants.ReplicaState;
@@ -68,11 +67,16 @@ class FinalizedReplica extends ReplicaInfo {
     detached = true;
   }
   
-  @Override  // ReplicaInfo
-  long getVisibleLen() throws IOException {
+  @Override
+  public long getVisibleLength() {
     return getNumBytes();       // all bytes are visible
   }
-  
+
+  @Override
+  public long getBytesOnDisk() {
+    return getNumBytes();
+  }
+
   @Override  // Object
   public boolean equals(Object o) {
     return super.equals(o);
@@ -82,4 +86,10 @@ class FinalizedReplica extends ReplicaInfo {
   public int hashCode() {
     return super.hashCode();
   }
+  
+  @Override
+  public String toString() {
+    return super.toString()
+        + "\n  detached=" + detached;
+  }
 }

+ 55 - 0
src/java/org/apache/hadoop/hdfs/server/datanode/Replica.java

@@ -0,0 +1,55 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.hadoop.hdfs.server.datanode;
+
+import org.apache.hadoop.hdfs.server.common.HdfsConstants.ReplicaState;
+
+/** 
+ * This represents block replicas which stored in DataNode.
+ */
+public interface Replica {
+  /** get block ID  */
+  public long getBlockId();
+
+  /** get generation stamp */
+  public long getGenerationStamp();
+
+  /**
+   * Get the replica state
+   * @return the replica state
+   */
+  public ReplicaState getState();
+
+  /**
+   * Get the number of bytes received
+   * @return the number of bytes that have been received
+   */
+  public long getNumBytes();
+  
+  /**
+   * Get the number of bytes that have written to disk
+   * @return the number of bytes that have written to disk
+   */
+  public long getBytesOnDisk();
+
+  /**
+   * Get the number of bytes that are visible to readers
+   * @return the number of bytes that are visible to readers
+   */
+  public long getVisibleLength();
+}

+ 3 - 3
src/java/org/apache/hadoop/hdfs/server/datanode/ReplicaBeingWritten.java

@@ -18,7 +18,7 @@
 package org.apache.hadoop.hdfs.server.datanode;
 
 import java.io.File;
-import java.io.IOException;
+
 import org.apache.hadoop.hdfs.protocol.Block;
 import org.apache.hadoop.hdfs.server.common.HdfsConstants.ReplicaState;
 import org.apache.hadoop.hdfs.server.datanode.FSDataset.FSVolume;
@@ -66,8 +66,8 @@ class ReplicaBeingWritten extends ReplicaInPipeline {
     super( blockId, len, genStamp, vol, dir, writer);
   }
   
-  @Override   //ReplicaInfo
-  long getVisibleLen() throws IOException {
+  @Override
+  public long getVisibleLength() {
     return getBytesAcked();       // all acked bytes are visible
   }
 

+ 11 - 5
src/java/org/apache/hadoop/hdfs/server/datanode/ReplicaInPipeline.java

@@ -85,10 +85,9 @@ class ReplicaInPipeline extends ReplicaInfo
     this.writer = writer;
   }
 
-  @Override  //ReplicaInfo
-  long getVisibleLen() throws IOException {
-    // no bytes are visible
-    throw new IOException("No bytes are visible for temporary replicas");
+  @Override
+  public long getVisibleLength() {
+    return -1;
   }
   
   @Override  //ReplicaInfo
@@ -172,5 +171,12 @@ class ReplicaInPipeline extends ReplicaInfo
       IOUtils.closeStream(crcOut);
       throw e;
     }
-  }  
+  }
+  
+  @Override
+  public String toString() {
+    return super.toString()
+        + "\n  bytesAcked=" + bytesAcked
+        + "\n  bytesOnDisk=" + bytesOnDisk;
+  }
 }

+ 1 - 13
src/java/org/apache/hadoop/hdfs/server/datanode/ReplicaInPipelineInterface.java

@@ -24,13 +24,7 @@ import org.apache.hadoop.hdfs.server.datanode.FSDatasetInterface.BlockWriteStrea
 /** 
  * This defines the interface of a replica in Pipeline that's being written to
  */
-interface ReplicaInPipelineInterface {
-  /**
-   * Get the number of bytes received
-   * @return the number of bytes that have been received
-   */
-  long getNumBytes();
-  
+interface ReplicaInPipelineInterface extends Replica {
   /**
    * Set the number of bytes received
    * @param bytesReceived number of bytes received
@@ -49,12 +43,6 @@ interface ReplicaInPipelineInterface {
    */
   void setBytesAcked(long bytesAcked);
   
-  /**
-   * Get the number of bytes that have written to disk
-   * @return the number of bytes that have written to disk
-   */
-  long getBytesOnDisk();
-  
   /**
    * Set the number of bytes on disk
    * @param bytesOnDisk number of bytes on disk

+ 9 - 17
src/java/org/apache/hadoop/hdfs/server/datanode/ReplicaInfo.java

@@ -25,7 +25,6 @@ import java.io.IOException;
 import org.apache.hadoop.fs.FileUtil;
 import org.apache.hadoop.fs.FileUtil.HardLink;
 import org.apache.hadoop.hdfs.protocol.Block;
-import org.apache.hadoop.hdfs.server.common.HdfsConstants.ReplicaState;
 import org.apache.hadoop.hdfs.server.datanode.FSDataset.FSVolume;
 import org.apache.hadoop.io.IOUtils;
 
@@ -33,7 +32,7 @@ import org.apache.hadoop.io.IOUtils;
  * This class is used by datanodes to maintain meta data of its replicas.
  * It provides a general interface for meta information of a replica.
  */
-abstract public class ReplicaInfo extends Block {
+abstract public class ReplicaInfo extends Block implements Replica {
   private FSVolume volume;      // volume where the replica belongs
   private File     dir;         // directory where block & meta files belong
 
@@ -129,13 +128,6 @@ abstract public class ReplicaInfo extends Block {
     this.dir = dir;
   }
 
-
-  /**
-   * Get the replica state
-   * @return the replica state
-   */
-  abstract public ReplicaState getState();
-  
   /**
    * check if this replica has already detached.
    * @return true if the replica has already detached or no need to detach; 
@@ -221,12 +213,6 @@ abstract public class ReplicaInfo extends Block {
     return true;
   }
 
-  /**
-   * Get the number of bytes that are visible to readers
-   * @return the number of bytes that are visible to readers
-   */
-  abstract long getVisibleLen() throws IOException;
-  
   /**
    * Set this replica's generation stamp to be a newer one
    * @param newGS new generation stamp
@@ -243,7 +229,13 @@ abstract public class ReplicaInfo extends Block {
   
   @Override  //Object
   public String toString() {
-    return getClass().getSimpleName() + " " + super.toString() + 
-    "(volume=" + volume + ", file=" + getBlockFile() + ")";
+    return getClass().getSimpleName()
+        + ", " + super.toString()
+        + ", " + getState()
+        + "\n  getNumBytes()     = " + getNumBytes()
+        + "\n  getBytesOnDisk()  = " + getBytesOnDisk()
+        + "\n  getVisibleLength()= " + getVisibleLength()
+        + "\n  getVolume()       = " + getVolume()
+        + "\n  getBlockFile()    = " + getBlockFile();
   }
 }

+ 14 - 3
src/java/org/apache/hadoop/hdfs/server/datanode/ReplicaUnderRecovery.java

@@ -18,7 +18,6 @@
 package org.apache.hadoop.hdfs.server.datanode;
 
 import java.io.File;
-import java.io.IOException;
 
 import org.apache.hadoop.hdfs.server.common.HdfsConstants.ReplicaState;
 import org.apache.hadoop.hdfs.server.datanode.FSDataset.FSVolume;
@@ -101,8 +100,13 @@ class ReplicaUnderRecovery extends ReplicaInfo {
   }
   
   @Override
-  long getVisibleLen() throws IOException {
-    return original.getVisibleLen();
+  public long getVisibleLength() {
+    return original.getVisibleLength();
+  }
+
+  @Override
+  public long getBytesOnDisk() {
+    return original.getBytesOnDisk();
   }
 
   @Override  //org.apache.hadoop.hdfs.protocol.Block
@@ -144,4 +148,11 @@ class ReplicaUnderRecovery extends ReplicaInfo {
   public int hashCode() {
     return super.hashCode();
   }
+
+  @Override
+  public String toString() {
+    return super.toString()
+        + "\n  recoveryId=" + recoveryId
+        + "\n  original=" + original;
+  }
 }

+ 12 - 2
src/java/org/apache/hadoop/hdfs/server/datanode/ReplicaWaitingToBeRecovered.java

@@ -18,7 +18,6 @@
 package org.apache.hadoop.hdfs.server.datanode;
 
 import java.io.File;
-import java.io.IOException;
 
 import org.apache.hadoop.hdfs.protocol.Block;
 import org.apache.hadoop.hdfs.server.common.HdfsConstants.ReplicaState;
@@ -75,10 +74,15 @@ class ReplicaWaitingToBeRecovered extends ReplicaInfo {
   }
   
   @Override //ReplicaInfo
-  long getVisibleLen() throws IOException {
+  public long getVisibleLength() {
     return -1;  //no bytes are visible
   }
   
+  @Override
+  public long getBytesOnDisk() {
+    return getNumBytes();
+  }
+
   @Override  // Object
   public boolean equals(Object o) {
     return super.equals(o);
@@ -88,4 +92,10 @@ class ReplicaWaitingToBeRecovered extends ReplicaInfo {
   public int hashCode() {
     return super.hashCode();
   }
+
+  @Override
+  public String toString() {
+    return super.toString()
+        + "\n  detached=" + detached;
+  }
 }

+ 22 - 1
src/test/hdfs/org/apache/hadoop/hdfs/server/datanode/SimulatedFSDataset.java

@@ -34,6 +34,7 @@ import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.hdfs.protocol.Block;
 import org.apache.hadoop.hdfs.protocol.BlockListAsLongs;
 import org.apache.hadoop.hdfs.protocol.FSConstants;
+import org.apache.hadoop.hdfs.server.common.HdfsConstants.ReplicaState;
 import org.apache.hadoop.hdfs.server.datanode.metrics.FSDatasetMBean;
 import org.apache.hadoop.metrics.util.MBeanUtil;
 import org.apache.hadoop.util.DataChecksum;
@@ -107,7 +108,7 @@ public class SimulatedFSDataset  implements FSConstants, FSDatasetInterface, Con
       }
     }
 
-    synchronized long getGenerationStamp() {
+    synchronized public long getGenerationStamp() {
       return theBlock.getGenerationStamp();
     }
 
@@ -209,6 +210,21 @@ public class SimulatedFSDataset  implements FSConstants, FSDatasetInterface, Con
       }
     }
 
+    @Override
+    synchronized public long getBlockId() {
+      return theBlock.getBlockId();
+    }
+
+    @Override
+    synchronized public long getVisibleLength() {
+      return getBytesAcked();
+    }
+
+    @Override
+    public ReplicaState getState() {
+      return null;
+    }
+
     @Override
     synchronized public long getBytesAcked() {
       if (finalized) {
@@ -379,6 +395,11 @@ public class SimulatedFSDataset  implements FSConstants, FSDatasetInterface, Con
     return binfo.getNumBytes();
   }
 
+  @Override
+  public Replica getReplica(long blockId) {
+    return blockMap.get(new Block(blockId));
+  }
+
   /** {@inheritDoc} */
   public Block getStoredBlock(long blkid) throws IOException {
     Block b = new Block(blkid);

+ 8 - 10
src/test/hdfs/org/apache/hadoop/hdfs/server/datanode/TestDirectoryScanner.java

@@ -317,23 +317,21 @@ public class TestDirectoryScanner extends TestCase {
   }
 
   private void verifyAddition(long blockId, long genStamp, long size) {
-    Block memBlock = fds.getBlock(blockId);
-    assertNotNull(memBlock);
-    ReplicaInfo blockInfo;
+    final ReplicaInfo replicainfo;
     synchronized(fds) {
-      blockInfo = fds.volumeMap.get(memBlock);
+      replicainfo = fds.getReplica(blockId);
     }
-    assertNotNull(blockInfo);
+    assertNotNull(replicainfo);
 
     // Added block has the same file as the one created by the test
     File file = new File(getBlockFile(blockId));
-    assertEquals(file.getName(), blockInfo.getBlockFile().getName());
+    assertEquals(file.getName(), replicainfo.getBlockFile().getName());
 
     // Generation stamp is same as that of created file
-    assertEquals(genStamp, memBlock.getGenerationStamp());
+    assertEquals(genStamp, replicainfo.getGenerationStamp());
 
     // File size matches
-    assertEquals(size, memBlock.getNumBytes());
+    assertEquals(size, replicainfo.getNumBytes());
   }
 
   private void verifyDeletion(long blockId) {
@@ -344,9 +342,9 @@ public class TestDirectoryScanner extends TestCase {
   }
 
   private void verifyGenStamp(long blockId, long genStamp) {
-    Block memBlock;
+    final Replica memBlock;
     synchronized(fds) {
-      memBlock = fds.getBlock(blockId);
+      memBlock = fds.getReplica(blockId);
     }
     assertNotNull(memBlock);
     assertEquals(genStamp, memBlock.getGenerationStamp());