Kaynağa Gözat

HDFS-11634. Optimize BlockIterator when interating starts in the middle. Contributed by Konstantin V Shvachko.

(cherry picked from commit 8dfcd95d580bb090af7f40af0a57061518c18c8c)
Konstantin V Shvachko 8 yıl önce
ebeveyn
işleme
8b3b3aac10

+ 1 - 5
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/blockmanagement/BlockManager.java

@@ -1184,13 +1184,9 @@ public class BlockManager implements BlockStatsMXBean {
     if(numBlocks == 0) {
       return new BlocksWithLocations(new BlockWithLocations[0]);
     }
-    Iterator<BlockInfo> iter = node.getBlockIterator();
     // starting from a random block
     int startBlock = ThreadLocalRandom.current().nextInt(numBlocks);
-    // skip blocks
-    for(int i=0; i<startBlock; i++) {
-      iter.next();
-    }
+    Iterator<BlockInfo> iter = node.getBlockIterator(startBlock);
     List<BlockWithLocations> results = new ArrayList<BlockWithLocations>();
     long totalSize = 0;
     BlockInfo curBlock;

+ 28 - 4
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/blockmanagement/DatanodeDescriptor.java

@@ -499,18 +499,35 @@ public class DatanodeDescriptor extends DatanodeInfo {
     private int index = 0;
     private final List<Iterator<BlockInfo>> iterators;
     
-    private BlockIterator(final DatanodeStorageInfo... storages) {
+    private BlockIterator(final int startBlock,
+                          final DatanodeStorageInfo... storages) {
+      if(startBlock < 0) {
+        throw new IllegalArgumentException(
+            "Illegal value startBlock = " + startBlock);
+      }
       List<Iterator<BlockInfo>> iterators = new ArrayList<>();
+      int s = startBlock;
+      int sumBlocks = 0;
       for (DatanodeStorageInfo e : storages) {
-        iterators.add(e.getBlockIterator());
+        int numBlocks = e.numBlocks();
+        sumBlocks += numBlocks;
+        if(sumBlocks <= startBlock) {
+          s -= numBlocks;
+        } else {
+          iterators.add(e.getBlockIterator());
+        }
       }
       this.iterators = Collections.unmodifiableList(iterators);
+      // skip to the storage containing startBlock
+      for(; s > 0 && hasNext(); s--) {
+        next();
+      }
     }
 
     @Override
     public boolean hasNext() {
       update();
-      return !iterators.isEmpty() && iterators.get(index).hasNext();
+      return index < iterators.size() && iterators.get(index).hasNext();
     }
 
     @Override
@@ -532,7 +549,14 @@ public class DatanodeDescriptor extends DatanodeInfo {
   }
 
   Iterator<BlockInfo> getBlockIterator() {
-    return new BlockIterator(getStorageInfos());
+    return getBlockIterator(0);
+  }
+
+  /**
+   * Get iterator, which starts iterating from the specified block.
+   */
+  Iterator<BlockInfo> getBlockIterator(final int startBlock) {
+    return new BlockIterator(startBlock, getStorageInfos());
   }
 
   void incrementPendingReplicationWithoutTargets() {

+ 70 - 7
hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/TestGetBlocks.java

@@ -22,6 +22,7 @@ import static org.junit.Assert.*;
 import java.io.IOException;
 import java.net.InetSocketAddress;
 import java.util.HashMap;
+import java.util.Iterator;
 import java.util.List;
 import java.util.Map;
 import java.util.Random;
@@ -38,9 +39,13 @@ import org.apache.hadoop.hdfs.protocol.HdfsConstants;
 import org.apache.hadoop.hdfs.protocol.HdfsConstants.DatanodeReportType;
 import org.apache.hadoop.hdfs.protocol.LocatedBlock;
 import org.apache.hadoop.hdfs.protocol.LocatedBlocks;
+import org.apache.hadoop.hdfs.server.blockmanagement.BlockInfo;
+import org.apache.hadoop.hdfs.server.blockmanagement.BlockManagerTestUtil;
 import org.apache.hadoop.hdfs.server.blockmanagement.DatanodeDescriptor;
+import org.apache.hadoop.hdfs.server.blockmanagement.DatanodeStorageInfo;
 import org.apache.hadoop.hdfs.server.datanode.DataNode;
 import org.apache.hadoop.hdfs.server.datanode.DataNodeTestUtils;
+import org.apache.hadoop.hdfs.server.namenode.FSNamesystem;
 import org.apache.hadoop.hdfs.server.protocol.BlocksWithLocations.BlockWithLocations;
 import org.apache.hadoop.hdfs.server.protocol.NamenodeProtocol;
 import org.apache.hadoop.ipc.RemoteException;
@@ -184,12 +189,14 @@ public class TestGetBlocks {
     CONF.setLong(DFSConfigKeys.DFS_BALANCER_GETBLOCKS_MIN_BLOCK_SIZE_KEY,
       DEFAULT_BLOCK_SIZE);
 
-    MiniDFSCluster cluster = new MiniDFSCluster.Builder(CONF).numDataNodes(
-        REPLICATION_FACTOR).build();
+    MiniDFSCluster cluster = new MiniDFSCluster.Builder(CONF)
+              .numDataNodes(REPLICATION_FACTOR)
+              .storagesPerDatanode(4)
+              .build();
     try {
       cluster.waitActive();
       // the third block will not be visible to getBlocks
-      long fileLen = 2 * DEFAULT_BLOCK_SIZE + 1;
+      long fileLen = 12 * DEFAULT_BLOCK_SIZE + 1;
       DFSTestUtil.createFile(cluster.getFileSystem(), new Path("/tmp.txt"),
           fileLen, REPLICATION_FACTOR, 0L);
 
@@ -197,12 +204,12 @@ public class TestGetBlocks {
       List<LocatedBlock> locatedBlocks;
       DatanodeInfo[] dataNodes = null;
       boolean notWritten;
+      final DFSClient dfsclient = new DFSClient(
+          DFSUtilClient.getNNAddress(CONF), CONF);
       do {
-        final DFSClient dfsclient = new DFSClient(
-            DFSUtilClient.getNNAddress(CONF), CONF);
         locatedBlocks = dfsclient.getNamenode()
             .getBlockLocations("/tmp.txt", 0, fileLen).getLocatedBlocks();
-        assertEquals(3, locatedBlocks.size());
+        assertEquals(13, locatedBlocks.size());
         notWritten = false;
         for (int i = 0; i < 2; i++) {
           dataNodes = locatedBlocks.get(i).getLocations();
@@ -216,6 +223,7 @@ public class TestGetBlocks {
           }
         }
       } while (notWritten);
+      dfsclient.close();
 
       // get RPC client to namenode
       InetSocketAddress addr = new InetSocketAddress("localhost",
@@ -226,7 +234,7 @@ public class TestGetBlocks {
       // get blocks of size fileLen from dataNodes[0]
       BlockWithLocations[] locs;
       locs = namenode.getBlocks(dataNodes[0], fileLen).getBlocks();
-      assertEquals(locs.length, 2);
+      assertEquals(locs.length, 12);
       assertEquals(locs[0].getStorageIDs().length, 2);
       assertEquals(locs[1].getStorageIDs().length, 2);
 
@@ -249,6 +257,8 @@ public class TestGetBlocks {
       // get blocks of size BlockSize from a non-existent datanode
       DatanodeInfo info = DFSTestUtil.getDatanodeInfo("1.2.3.4");
       getBlocksWithException(namenode, info, 2);
+
+      testBlockIterator(cluster);
     } finally {
       cluster.shutdown();
     }
@@ -266,6 +276,59 @@ public class TestGetBlocks {
     assertTrue(getException);
   }
 
+  /**
+   * BlockIterator iterates over all blocks belonging to DatanodeDescriptor
+   * through multiple storages.
+   * The test verifies that BlockIterator can be set to start iterating from
+   * a particular starting block index.
+   */
+  void testBlockIterator(MiniDFSCluster cluster) {
+    FSNamesystem ns = cluster.getNamesystem();
+    String dId = cluster.getDataNodes().get(0).getDatanodeUuid();
+    DatanodeDescriptor dnd = BlockManagerTestUtil.getDatanode(ns, dId);
+    DatanodeStorageInfo[] storages = dnd.getStorageInfos();
+    assertEquals("DataNode should have 4 storages", 4, storages.length);
+
+    Iterator<BlockInfo> dnBlockIt = null;
+    // check illegal start block number
+    try {
+      dnBlockIt = BlockManagerTestUtil.getBlockIterator(
+          cluster.getNamesystem(), dId, -1);
+      assertTrue("Should throw IllegalArgumentException", false);
+    } catch(IllegalArgumentException ei) {
+      // as expected
+    }
+    assertNull("Iterator should be null", dnBlockIt);
+
+    // form an array of all DataNode blocks
+    int numBlocks = dnd.numBlocks();
+    BlockInfo[] allBlocks = new BlockInfo[numBlocks];
+    int idx = 0;
+    for(DatanodeStorageInfo s : storages) {
+      Iterator<BlockInfo> storageBlockIt =
+          BlockManagerTestUtil.getBlockIterator(s);
+      while(storageBlockIt.hasNext()) {
+        allBlocks[idx++] = storageBlockIt.next();
+      }
+    }
+
+    // check iterator for every block as a starting point
+    for(int i = 0; i < allBlocks.length; i++) {
+      // create iterator starting from i
+      dnBlockIt = BlockManagerTestUtil.getBlockIterator(ns, dId, i);
+      assertTrue("Block iterator should have next block", dnBlockIt.hasNext());
+      // check iterator lists blocks in the desired order
+      for(int j = i; j < allBlocks.length; j++) {
+        assertEquals("Wrong block order", allBlocks[j], dnBlockIt.next());
+      }
+    }
+
+    // check start block number larger than numBlocks in the DataNode
+    dnBlockIt = BlockManagerTestUtil.getBlockIterator(
+        ns, dId, allBlocks.length + 1);
+    assertFalse("Iterator should not have next block", dnBlockIt.hasNext());
+  }
+
   @Test
   public void testBlockKey() {
     Map<Block, Long> map = new HashMap<Block, Long>();

+ 16 - 0
hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/server/blockmanagement/BlockManagerTestUtil.java

@@ -21,6 +21,7 @@ import java.io.IOException;
 import java.util.ArrayList;
 import java.util.Collection;
 import java.util.HashSet;
+import java.util.Iterator;
 import java.util.Set;
 import java.util.concurrent.ExecutionException;
 
@@ -53,6 +54,21 @@ public class BlockManagerTestUtil {
     }
   }
 
+  public static Iterator<BlockInfo> getBlockIterator(final FSNamesystem ns,
+      final String storageID, final int startBlock) {
+    ns.readLock();
+    try {
+      DatanodeDescriptor dn =
+          ns.getBlockManager().getDatanodeManager().getDatanode(storageID);
+      return dn.getBlockIterator(startBlock);
+    } finally {
+      ns.readUnlock();
+    }
+  }
+
+  public static Iterator<BlockInfo> getBlockIterator(DatanodeStorageInfo s) {
+    return s.getBlockIterator();
+  }
 
   /**
    * Refresh block queue counts on the name-node.