浏览代码

Remove getPreferredBlockReplication().

Haohui Mai 10 年之前
父节点
当前提交
cf23b98670

+ 0 - 6
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/blockmanagement/BlockCollection.java

@@ -55,12 +55,6 @@ public interface BlockCollection {
   public long getPreferredBlockSize();
 
   /**
-   * Get block replication for the collection 
-   * @return block replication value
-   */
-  public short getPreferredBlockReplication();
-
-  /** 
    * @return the storage policy ID.
    */
   public byte getStoragePolicyID();

+ 1 - 1
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/FSDirWriteFileOp.java

@@ -87,7 +87,7 @@ class FSDirWriteFileOp {
 
     // update space consumed
     fsd.updateCount(iip, 0, -fileNode.getPreferredBlockSize(),
-                    fileNode.getPreferredBlockReplication(), true);
+                    uc.getReplication(), true);
     return true;
   }
 

+ 2 - 17
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/INodeFile.java

@@ -354,20 +354,6 @@ public class INodeFile extends INodeWithAdditionalFields
     return getFileReplication(CURRENT_STATE_ID);
   }
 
-  @Override // BlockCollection
-  public short getPreferredBlockReplication() {
-    short max = getFileReplication(CURRENT_STATE_ID);
-    FileWithSnapshotFeature sf = this.getFileWithSnapshotFeature();
-    if (sf != null) {
-      short maxInSnapshot = sf.getMaxBlockRepInDiffs();
-      if (sf.isCurrentFileDeleted()) {
-        return maxInSnapshot;
-      }
-      max = maxInSnapshot > max ? maxInSnapshot : max;
-    }
-    return max;
-  }
-
   /** Set the replication factor of this file. */
   public final void setFileReplication(short replication) {
     header = HeaderFormat.REPLICATION.BITS.combine(replication, header);
@@ -404,7 +390,7 @@ public class INodeFile extends INodeWithAdditionalFields
 
   private void setStoragePolicyID(byte storagePolicyId) {
     header = HeaderFormat.STORAGE_POLICY_ID.BITS.combine(storagePolicyId,
-        header);
+                                                         header);
   }
 
   public final void setStoragePolicyID(byte storagePolicyId,
@@ -851,8 +837,7 @@ public class INodeFile extends INodeWithAdditionalFields
 
       delta.addStorageSpace(-truncatedBytes * bi.getReplication());
       if (bsps != null) {
-        List<StorageType> types = bsps.chooseStorageTypes(
-            getPreferredBlockReplication());
+        List<StorageType> types = bsps.chooseStorageTypes(bi.getReplication());
         for (StorageType t : types) {
           if (t.supportTypeQuota()) {
             delta.addTypeSpace(t, -truncatedBytes);

+ 27 - 11
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/snapshot/FileWithSnapshotFeature.java

@@ -17,8 +17,11 @@
  */
 package org.apache.hadoop.hdfs.server.namenode.snapshot;
 
+import java.util.Arrays;
+import java.util.HashSet;
 import java.util.List;
 
+import com.google.common.collect.Sets;
 import org.apache.hadoop.classification.InterfaceAudience;
 import org.apache.hadoop.fs.StorageType;
 import org.apache.hadoop.hdfs.protocol.HdfsConstants;
@@ -150,23 +153,34 @@ public class FileWithSnapshotFeature implements INode.Feature {
       bsp = reclaimContext.storagePolicySuite().getPolicy(file.getStoragePolicyID());
     }
 
-    QuotaCounts oldCounts = file.storagespaceConsumed(null);
+    QuotaCounts oldCounts;
     if (removed.snapshotINode != null) {
-      short replication = removed.snapshotINode.getFileReplication();
-      short currentRepl = file.getPreferredBlockReplication();
-      if (replication > currentRepl) {
-        long oldFileSizeNoRep = currentRepl == 0
-            ? file.computeFileSize(true, true)
-            : oldCounts.getStorageSpace() /
-            file.getPreferredBlockReplication();
-        long oldStoragespace = oldFileSizeNoRep * replication;
-        oldCounts.setStorageSpace(oldStoragespace);
+      // Assuming the replication factor of blocks have been changed -- here
+      // we recompute the quota as if the old inode is around
+      INodeFile removedINode = (INodeFile) removed.getSnapshotINode();
+      HashSet<BlockInfoContiguous> oldBlocks = new HashSet<>();
+      if (removedINode.getBlocks() != null) {
+        oldBlocks.addAll(Arrays.asList(removedINode.getBlocks()));
+      }
+
+      oldCounts = new QuotaCounts.Builder().build();
+      BlockInfoContiguous[] blocks = file.getBlocks() == null ? new
+          BlockInfoContiguous[0] : file.getBlocks();
+      short oldReplication = removed.snapshotINode.getFileReplication();
+      for (BlockInfoContiguous b: blocks) {
+        short replication = b.getReplication();
+        if (oldBlocks.contains(b) && oldReplication > b.getReplication()) {
+          replication = b.getReplication();
+        }
+        long blockSize = b.isComplete() ? b.getNumBytes() : file
+            .getPreferredBlockSize();
 
+        oldCounts.addStorageSpace(blockSize * replication);
         if (bsp != null) {
           List<StorageType> oldTypeChosen = bsp.chooseStorageTypes(replication);
           for (StorageType t : oldTypeChosen) {
             if (t.supportTypeQuota()) {
-              oldCounts.addTypeSpace(t, oldFileSizeNoRep);
+              oldCounts.addTypeSpace(t, blockSize);
             }
           }
         }
@@ -176,6 +190,8 @@ public class FileWithSnapshotFeature implements INode.Feature {
       if (aclFeature != null) {
         AclStorage.removeAclFeature(aclFeature);
       }
+    } else {
+      oldCounts = file.storagespaceConsumed(null);
     }
 
     getDiffs().combineAndCollectSnapshotBlocks(reclaimContext, file, removed);

+ 0 - 2
hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/server/blockmanagement/TestBlockManager.java

@@ -739,7 +739,6 @@ public class TestBlockManager {
     BlockInfoContiguous blockInfo =
         new BlockInfoContiguous(block, (short) 3);
     BlockCollection bc = Mockito.mock(BlockCollection.class);
-    Mockito.doReturn((short) 3).when(bc).getPreferredBlockReplication();
     bm.blocksMap.addBlockCollection(blockInfo, bc);
     return blockInfo;
   }
@@ -749,7 +748,6 @@ public class TestBlockManager {
     BlockInfoContiguousUnderConstruction blockInfo =
         new BlockInfoContiguousUnderConstruction(block, (short) 3);
     BlockCollection bc = Mockito.mock(BlockCollection.class);
-    Mockito.doReturn((short) 3).when(bc).getPreferredBlockReplication();
     bm.blocksMap.addBlockCollection(blockInfo, bc);
     return blockInfo;
   }

+ 0 - 1
hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/server/namenode/snapshot/TestFileWithSnapshotFeature.java

@@ -71,7 +71,6 @@ public class TestFileWithSnapshotFeature {
 
     // INode only exists in the snapshot
     INodeFile snapshotINode = mock(INodeFile.class);
-    when(file.getPreferredBlockReplication()).thenReturn(REPL_1);
     Whitebox.setInternalState(snapshotINode, "header", (long) REPL_3 << 48);
     Whitebox.setInternalState(diff, "snapshotINode", snapshotINode);
     when(diff.getSnapshotINode()).thenReturn(snapshotINode);

+ 7 - 2
hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/server/namenode/snapshot/TestSnapshotDeletion.java

@@ -843,12 +843,17 @@ public class TestSnapshotDeletion {
     }
     
     INodeFile nodeFile13 = (INodeFile) fsdir.getINode(file13.toString());
-    assertEquals(REPLICATION_1, nodeFile13.getPreferredBlockReplication());
+    for (BlockInfoContiguous b : nodeFile13.getBlocks()) {
+      assertEquals(REPLICATION_1, b.getReplication());
+    }
+
     TestSnapshotBlocksMap.assertBlockCollection(file13.toString(), 1, fsdir,
         blockmanager);
     
     INodeFile nodeFile12 = (INodeFile) fsdir.getINode(file12_s1.toString());
-    assertEquals(REPLICATION_1, nodeFile12.getPreferredBlockReplication());
+    for (BlockInfoContiguous b : nodeFile12.getBlocks()) {
+      assertEquals(REPLICATION_1, b.getReplication());
+    }
   }
   
   /** Test deleting snapshots with modification on the metadata of directory */ 

+ 17 - 13
hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/server/namenode/snapshot/TestSnapshotReplication.java

@@ -28,6 +28,7 @@ import org.apache.hadoop.fs.Path;
 import org.apache.hadoop.hdfs.DFSTestUtil;
 import org.apache.hadoop.hdfs.DistributedFileSystem;
 import org.apache.hadoop.hdfs.MiniDFSCluster;
+import org.apache.hadoop.hdfs.server.blockmanagement.BlockInfoContiguous;
 import org.apache.hadoop.hdfs.server.namenode.FSDirectory;
 import org.apache.hadoop.hdfs.server.namenode.FSNamesystem;
 import org.apache.hadoop.hdfs.server.namenode.INode;
@@ -38,10 +39,9 @@ import org.junit.Before;
 import org.junit.Test;
 
 /**
- * This class tests the replication handling/calculation of snapshots. In
- * particular, {@link INodeFile#getFileReplication()} and
- * {@link INodeFile#getPreferredBlockReplication()} are tested to make sure
- * the number of replication is calculated correctly with/without snapshots.
+ * This class tests the replication handling/calculation of snapshots to make
+ * sure the number of replication is calculated correctly with/without
+ * snapshots.
  */
 public class TestSnapshotReplication {
   
@@ -79,9 +79,7 @@ public class TestSnapshotReplication {
   }
   
   /**
-   * Check the replication of a given file. We test both
-   * {@link INodeFile#getFileReplication()} and
-   * {@link INodeFile#getPreferredBlockReplication()}.
+   * Check the replication of a given file.
    *
    * @param file The given file
    * @param replication The expected replication number
@@ -98,8 +96,9 @@ public class TestSnapshotReplication {
     // Check the correctness of getPreferredBlockReplication()
     INode inode = fsdir.getINode(file1.toString());
     assertTrue(inode instanceof INodeFile);
-    assertEquals(blockReplication,
-        ((INodeFile) inode).getPreferredBlockReplication());
+    for (BlockInfoContiguous b : inode.asFile().getBlocks()) {
+      assertEquals(blockReplication, b.getReplication());
+    }
   }
   
   /**
@@ -141,8 +140,9 @@ public class TestSnapshotReplication {
     // First check the getPreferredBlockReplication for the INode of
     // the currentFile
     final INodeFile inodeOfCurrentFile = getINodeFile(currentFile);
-    assertEquals(expectedBlockRep,
-        inodeOfCurrentFile.getPreferredBlockReplication());
+    for (BlockInfoContiguous b : inodeOfCurrentFile.getBlocks()) {
+      assertEquals(expectedBlockRep, b.getReplication());
+    }
     // Then check replication for every snapshot
     for (Path ss : snapshotRepMap.keySet()) {
       final INodesInPath iip = fsdir.getINodesInPath(ss.toString(), true);
@@ -150,7 +150,9 @@ public class TestSnapshotReplication {
       // The replication number derived from the
       // INodeFileWithLink#getPreferredBlockReplication should
       // always == expectedBlockRep
-      assertEquals(expectedBlockRep, ssInode.getPreferredBlockReplication());
+      for (BlockInfoContiguous b : ssInode.getBlocks()) {
+        assertEquals(expectedBlockRep, b.getReplication());
+      }
       // Also check the number derived from INodeFile#getFileReplication
       assertEquals(snapshotRepMap.get(ss).shortValue(),
           ssInode.getFileReplication(iip.getPathSnapshotId()));
@@ -224,7 +226,9 @@ public class TestSnapshotReplication {
       // The replication number derived from the
       // INodeFileWithLink#getPreferredBlockReplication should
       // always == expectedBlockRep
-      assertEquals(REPLICATION, ssInode.getPreferredBlockReplication());
+      for (BlockInfoContiguous b : ssInode.getBlocks()) {
+        assertEquals(REPLICATION, b.getReplication());
+      }
       // Also check the number derived from INodeFile#getFileReplication
       assertEquals(snapshotRepMap.get(ss).shortValue(),
           ssInode.getFileReplication());