Return-Path: X-Original-To: apmail-hadoop-common-commits-archive@www.apache.org Delivered-To: apmail-hadoop-common-commits-archive@www.apache.org Received: from mail.apache.org (hermes.apache.org [140.211.11.3]) by minotaur.apache.org (Postfix) with SMTP id 2837C18F44 for ; Fri, 14 Aug 2015 10:54:17 +0000 (UTC) Received: (qmail 93295 invoked by uid 500); 14 Aug 2015 10:54:15 -0000 Delivered-To: apmail-hadoop-common-commits-archive@hadoop.apache.org Received: (qmail 93101 invoked by uid 500); 14 Aug 2015 10:54:15 -0000 Mailing-List: contact common-commits-help@hadoop.apache.org; run by ezmlm Precedence: bulk List-Help: List-Unsubscribe: List-Post: List-Id: Reply-To: common-dev@hadoop.apache.org Delivered-To: mailing list common-commits@hadoop.apache.org Received: (qmail 93065 invoked by uid 99); 14 Aug 2015 10:54:15 -0000 Received: from git1-us-west.apache.org (HELO git1-us-west.apache.org) (140.211.11.23) by apache.org (qpsmtpd/0.29) with ESMTP; Fri, 14 Aug 2015 10:54:15 +0000 Received: by git1-us-west.apache.org (ASF Mail Server at git1-us-west.apache.org, from userid 33) id A8DA0E35C6; Fri, 14 Aug 2015 10:54:15 +0000 (UTC) Content-Type: text/plain; charset="us-ascii" MIME-Version: 1.0 Content-Transfer-Encoding: 7bit From: vinayakumarb@apache.org To: common-commits@hadoop.apache.org Date: Fri, 14 Aug 2015 10:54:19 -0000 Message-Id: <5a28e627dda64fd5946dcfc8dfedaadc@git.apache.org> In-Reply-To: <0e8188f9427546bfa5c0a42a5f7f5505@git.apache.org> References: <0e8188f9427546bfa5c0a42a5f7f5505@git.apache.org> X-Mailer: ASF-Git Admin Mailer Subject: [05/50] hadoop git commit: HDFS-8453. Erasure coding: properly handle start offset for internal blocks in a block group. Contributed by Zhe Zhang. HDFS-8453. Erasure coding: properly handle start offset for internal blocks in a block group. Contributed by Zhe Zhang. Project: http://git-wip-us.apache.org/repos/asf/hadoop/repo Commit: http://git-wip-us.apache.org/repos/asf/hadoop/commit/d8069875 Tree: http://git-wip-us.apache.org/repos/asf/hadoop/tree/d8069875 Diff: http://git-wip-us.apache.org/repos/asf/hadoop/diff/d8069875 Branch: refs/heads/HDFS-7285-REBASE Commit: d80698751500ae3353fe2161933724267b2e585b Parents: 2adb9dc Author: Jing Zhao Authored: Tue Jun 2 16:14:08 2015 -0700 Committer: Vinayakumar B Committed: Thu Aug 13 17:01:32 2015 +0530 ---------------------------------------------------------------------- .../hadoop/hdfs/protocol/HdfsConstants.java | 2 +- .../hadoop-hdfs/CHANGES-HDFS-EC-7285.txt | 3 ++ .../hadoop/hdfs/DFSStripedInputStream.java | 45 +++++++++----------- .../hadoop/hdfs/util/StripedBlockUtil.java | 5 +-- .../hadoop/hdfs/TestDFSStripedInputStream.java | 4 +- .../hdfs/TestReadStripedFileWithDecoding.java | 24 ++++++----- 6 files changed, 43 insertions(+), 40 deletions(-) ---------------------------------------------------------------------- http://git-wip-us.apache.org/repos/asf/hadoop/blob/d8069875/hadoop-hdfs-project/hadoop-hdfs-client/src/main/java/org/apache/hadoop/hdfs/protocol/HdfsConstants.java ---------------------------------------------------------------------- diff --git a/hadoop-hdfs-project/hadoop-hdfs-client/src/main/java/org/apache/hadoop/hdfs/protocol/HdfsConstants.java b/hadoop-hdfs-project/hadoop-hdfs-client/src/main/java/org/apache/hadoop/hdfs/protocol/HdfsConstants.java index 2fa4808..fa816e4 100644 --- a/hadoop-hdfs-project/hadoop-hdfs-client/src/main/java/org/apache/hadoop/hdfs/protocol/HdfsConstants.java +++ b/hadoop-hdfs-project/hadoop-hdfs-client/src/main/java/org/apache/hadoop/hdfs/protocol/HdfsConstants.java @@ -87,7 +87,7 @@ public final class HdfsConstants { public static final byte NUM_DATA_BLOCKS = 6; public static final byte NUM_PARITY_BLOCKS = 3; // The chunk size for striped block which is used by erasure coding - public static final int BLOCK_STRIPED_CELL_SIZE = 256 * 1024; + public static final int BLOCK_STRIPED_CELL_SIZE = 64 * 1024; // SafeMode actions public enum SafeModeAction { http://git-wip-us.apache.org/repos/asf/hadoop/blob/d8069875/hadoop-hdfs-project/hadoop-hdfs/CHANGES-HDFS-EC-7285.txt ---------------------------------------------------------------------- diff --git a/hadoop-hdfs-project/hadoop-hdfs/CHANGES-HDFS-EC-7285.txt b/hadoop-hdfs-project/hadoop-hdfs/CHANGES-HDFS-EC-7285.txt index fa0a8e2..278f897 100755 --- a/hadoop-hdfs-project/hadoop-hdfs/CHANGES-HDFS-EC-7285.txt +++ b/hadoop-hdfs-project/hadoop-hdfs/CHANGES-HDFS-EC-7285.txt @@ -274,3 +274,6 @@ HDFS-8517. Fix a decoding issue in stripped block recovering in client side. (Kai Zheng via jing9) + + HDFS-8453. Erasure coding: properly handle start offset for internal blocks + in a block group. (Zhe Zhang via jing9) http://git-wip-us.apache.org/repos/asf/hadoop/blob/d8069875/hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/DFSStripedInputStream.java ---------------------------------------------------------------------- diff --git a/hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/DFSStripedInputStream.java b/hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/DFSStripedInputStream.java index 228368b..2e26cca 100644 --- a/hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/DFSStripedInputStream.java +++ b/hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/DFSStripedInputStream.java @@ -26,6 +26,7 @@ import org.apache.hadoop.hdfs.protocol.ExtendedBlock; import org.apache.hadoop.hdfs.protocol.LocatedBlock; import org.apache.hadoop.hdfs.protocol.LocatedStripedBlock; import org.apache.hadoop.hdfs.protocol.datatransfer.InvalidEncryptionKeyException; +import org.apache.hadoop.hdfs.server.blockmanagement.BlockIdManager; import org.apache.hadoop.hdfs.util.StripedBlockUtil; import org.apache.hadoop.io.ByteBufferPool; @@ -470,22 +471,17 @@ public class DFSStripedInputStream extends DFSInputStream { } /** - * | <--------- LocatedStripedBlock (ID = 0) ---------> | - * LocatedBlock (0) | LocatedBlock (1) | LocatedBlock (2) - * ^ - * offset - * On a striped file, the super method {@link DFSInputStream#getBlockAt} - * treats a striped block group as a single {@link LocatedBlock} object, - * which includes target in its range. This method adds the logic of: - * 1. Analyzing the index of required block based on offset - * 2. Parsing the block group to obtain the block location on that index + * The super method {@link DFSInputStream#refreshLocatedBlock} refreshes + * cached LocatedBlock by executing {@link DFSInputStream#getBlockAt} again. + * This method extends the logic by first remembering the index of the + * internal block, and re-parsing the refreshed block group with the same + * index. */ @Override - protected LocatedBlock getBlockAt(long blkStartOffset) throws IOException { - LocatedBlock lb = getBlockGroupAt(blkStartOffset); - - int idx = (int) ((blkStartOffset - lb.getStartOffset()) - % (dataBlkNum + parityBlkNum)); + protected LocatedBlock refreshLocatedBlock(LocatedBlock block) + throws IOException { + int idx = BlockIdManager.getBlockIndex(block.getBlock().getLocalBlock()); + LocatedBlock lb = getBlockGroupAt(block.getStartOffset()); // If indexing information is returned, iterate through the index array // to find the entry for position idx in the group LocatedStripedBlock lsb = (LocatedStripedBlock) lb; @@ -496,10 +492,11 @@ public class DFSStripedInputStream extends DFSInputStream { } } if (DFSClient.LOG.isDebugEnabled()) { - DFSClient.LOG.debug("getBlockAt for striped blocks, offset=" - + blkStartOffset + ". Obtained block " + lb + ", idx=" + idx); + DFSClient.LOG.debug("refreshLocatedBlock for striped blocks, offset=" + + block.getStartOffset() + ". Obtained block " + lb + ", idx=" + idx); } - return StripedBlockUtil.constructInternalBlock(lsb, i, cellSize, dataBlkNum, idx); + return StripedBlockUtil.constructInternalBlock( + lsb, i, cellSize, dataBlkNum, idx); } private LocatedStripedBlock getBlockGroupAt(long offset) throws IOException { @@ -513,12 +510,12 @@ public class DFSStripedInputStream extends DFSInputStream { * Real implementation of pread. */ @Override - protected void fetchBlockByteRange(long blockStartOffset, long start, + protected void fetchBlockByteRange(LocatedBlock block, long start, long end, byte[] buf, int offset, Map> corruptedBlockMap) throws IOException { // Refresh the striped block group - LocatedStripedBlock blockGroup = getBlockGroupAt(blockStartOffset); + LocatedStripedBlock blockGroup = getBlockGroupAt(block.getStartOffset()); AlignedStripe[] stripes = divideByteRangeIntoStripes(schema, cellSize, blockGroup, start, end, buf, offset); @@ -622,9 +619,9 @@ public class DFSStripedInputStream extends DFSInputStream { StripingChunk chunk = alignedStripe.chunks[index]; chunk.state = StripingChunk.PENDING; Callable readCallable = getFromOneDataNode(dnAddr, - block.getStartOffset(), alignedStripe.getOffsetInBlock(), - alignedStripe.getOffsetInBlock() + alignedStripe.getSpanInBlock() - 1, chunk.buf, - chunk.getOffsets(), chunk.getLengths(), + block, alignedStripe.getOffsetInBlock(), + alignedStripe.getOffsetInBlock() + alignedStripe.getSpanInBlock() - 1, + chunk.buf, chunk.getOffsets(), chunk.getLengths(), corruptedBlockMap, index); Future getFromDNRequest = service.submit(readCallable); if (DFSClient.LOG.isDebugEnabled()) { @@ -637,7 +634,7 @@ public class DFSStripedInputStream extends DFSInputStream { } private Callable getFromOneDataNode(final DNAddrPair datanode, - final long blockStartOffset, final long start, final long end, + final LocatedBlock block, final long start, final long end, final byte[] buf, final int[] offsets, final int[] lengths, final Map> corruptedBlockMap, final int hedgedReadId) { @@ -648,7 +645,7 @@ public class DFSStripedInputStream extends DFSInputStream { TraceScope scope = Trace.startSpan("Parallel reading " + hedgedReadId, parentSpan); try { - actualGetFromOneDataNode(datanode, blockStartOffset, start, + actualGetFromOneDataNode(datanode, block, start, end, buf, offsets, lengths, corruptedBlockMap); } finally { scope.close(); http://git-wip-us.apache.org/repos/asf/hadoop/blob/d8069875/hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/util/StripedBlockUtil.java ---------------------------------------------------------------------- diff --git a/hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/util/StripedBlockUtil.java b/hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/util/StripedBlockUtil.java index 80321ef..1db2045 100644 --- a/hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/util/StripedBlockUtil.java +++ b/hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/util/StripedBlockUtil.java @@ -105,16 +105,15 @@ public class StripedBlockUtil { final ExtendedBlock blk = constructInternalBlock( bg.getBlock(), cellSize, dataBlkNum, idxInBlockGroup); - final long offset = bg.getStartOffset() + idxInBlockGroup * (long) cellSize; if (idxInReturnedLocs < bg.getLocations().length) { return new LocatedBlock(blk, new DatanodeInfo[]{bg.getLocations()[idxInReturnedLocs]}, new String[]{bg.getStorageIDs()[idxInReturnedLocs]}, new StorageType[]{bg.getStorageTypes()[idxInReturnedLocs]}, - offset, bg.isCorrupt(), null); + bg.getStartOffset(), bg.isCorrupt(), null); } else { return new LocatedBlock(blk, null, null, null, - offset, bg.isCorrupt(), null); + bg.getStartOffset(), bg.isCorrupt(), null); } } http://git-wip-us.apache.org/repos/asf/hadoop/blob/d8069875/hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/TestDFSStripedInputStream.java ---------------------------------------------------------------------- diff --git a/hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/TestDFSStripedInputStream.java b/hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/TestDFSStripedInputStream.java index b64e690..de43441 100644 --- a/hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/TestDFSStripedInputStream.java +++ b/hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/TestDFSStripedInputStream.java @@ -87,7 +87,7 @@ public class TestDFSStripedInputStream { * Test {@link DFSStripedInputStream#getBlockAt(long)} */ @Test - public void testGetBlock() throws Exception { + public void testRefreshBlock() throws Exception { final int numBlocks = 4; DFSTestUtil.createStripedFile(cluster, filePath, null, numBlocks, NUM_STRIPE_PER_BLOCK, false); @@ -102,7 +102,7 @@ public class TestDFSStripedInputStream { LocatedBlock[] blks = StripedBlockUtil.parseStripedBlockGroup(lsb, CELLSIZE, DATA_BLK_NUM, PARITY_BLK_NUM); for (int j = 0; j < DATA_BLK_NUM; j++) { - LocatedBlock refreshed = in.getBlockAt(blks[j].getStartOffset()); + LocatedBlock refreshed = in.refreshLocatedBlock(blks[j]); assertEquals(blks[j].getBlock(), refreshed.getBlock()); assertEquals(blks[j].getStartOffset(), refreshed.getStartOffset()); assertArrayEquals(blks[j].getLocations(), refreshed.getLocations()); http://git-wip-us.apache.org/repos/asf/hadoop/blob/d8069875/hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/TestReadStripedFileWithDecoding.java ---------------------------------------------------------------------- diff --git a/hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/TestReadStripedFileWithDecoding.java b/hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/TestReadStripedFileWithDecoding.java index 7397caf..a28f88e 100644 --- a/hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/TestReadStripedFileWithDecoding.java +++ b/hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/TestReadStripedFileWithDecoding.java @@ -59,20 +59,24 @@ public class TestReadStripedFileWithDecoding { @Test public void testWritePreadWithDNFailure1() throws IOException { - testWritePreadWithDNFailure("/foo", 0); + testWritePreadWithDNFailure("/foo", cellSize * (dataBlocks + 2), 0); } @Test public void testWritePreadWithDNFailure2() throws IOException { - testWritePreadWithDNFailure("/foo", cellSize * 5); + testWritePreadWithDNFailure("/foo", cellSize * (dataBlocks + 2), cellSize * 5); } - private void testWritePreadWithDNFailure(String file, int startOffsetInFile) + @Test + public void testWritePreadWithDNFailure3() throws IOException { + testWritePreadWithDNFailure("/foo", cellSize * dataBlocks, 0); + } + + private void testWritePreadWithDNFailure(String file, int fileSize, int startOffsetInFile) throws IOException { final int failedDNIdx = 2; - final int length = cellSize * (dataBlocks + 2); Path testPath = new Path(file); - final byte[] bytes = StripedFileTestUtil.generateBytes(length); + final byte[] bytes = StripedFileTestUtil.generateBytes(fileSize); DFSTestUtil.writeFile(fs, testPath, bytes); // shut down the DN that holds the last internal data block @@ -89,17 +93,17 @@ public class TestReadStripedFileWithDecoding { // pread try (FSDataInputStream fsdis = fs.open(testPath)) { - byte[] buf = new byte[length]; + byte[] buf = new byte[fileSize]; int readLen = fsdis.read(startOffsetInFile, buf, 0, buf.length); - Assert.assertEquals("The length of file should be the same to write size", - length - startOffsetInFile, readLen); + Assert.assertEquals("The fileSize of file should be the same to write size", + fileSize - startOffsetInFile, readLen); byte[] expected = new byte[readLen]; - for (int i = startOffsetInFile; i < length; i++) { + for (int i = startOffsetInFile; i < fileSize; i++) { expected[i - startOffsetInFile] = StripedFileTestUtil.getByte(i); } - for (int i = startOffsetInFile; i < length; i++) { + for (int i = startOffsetInFile; i < fileSize; i++) { Assert.assertEquals("Byte at " + i + " should be the same", expected[i - startOffsetInFile], buf[i - startOffsetInFile]); }