|
17 | 17 | */ |
18 | 18 | package org.apache.hadoop.hdfs; |
19 | 19 |
|
| 20 | +import static org.junit.Assert.assertTrue; |
| 21 | + |
20 | 22 | import java.io.IOException; |
| 23 | +import java.util.Arrays; |
| 24 | +import java.util.List; |
| 25 | +import java.util.Random; |
| 26 | +import java.util.concurrent.atomic.AtomicBoolean; |
21 | 27 |
|
22 | 28 | import com.google.common.base.Supplier; |
| 29 | + |
23 | 30 | import org.apache.hadoop.conf.Configuration; |
24 | 31 | import org.apache.hadoop.fs.FSDataInputStream; |
25 | 32 | import org.apache.hadoop.fs.FSDataOutputStream; |
|
31 | 38 | import org.apache.hadoop.hdfs.server.common.HdfsServerConstants.BlockUCState; |
32 | 39 | import org.apache.hadoop.hdfs.server.datanode.DataNode; |
33 | 40 | import org.apache.hadoop.hdfs.server.datanode.DataNodeFaultInjector; |
| 41 | +import org.apache.hadoop.hdfs.server.datanode.DataNodeTestUtils; |
| 42 | +import org.apache.hadoop.hdfs.server.datanode.ReplicaInPipelineInterface; |
34 | 43 | import org.apache.hadoop.hdfs.server.namenode.LeaseExpiredException; |
35 | 44 | import org.apache.hadoop.hdfs.server.protocol.NamenodeProtocols; |
36 | 45 | import org.apache.hadoop.hdfs.tools.DFSAdmin; |
|
39 | 48 | import org.junit.Assert; |
40 | 49 | import org.junit.Test; |
41 | 50 | import org.mockito.Mockito; |
| 51 | +import org.slf4j.Logger; |
| 52 | +import org.slf4j.LoggerFactory; |
42 | 53 |
|
43 | 54 | /** |
44 | 55 | * This tests pipeline recovery related client protocol works correct or not. |
45 | 56 | */ |
46 | 57 | public class TestClientProtocolForPipelineRecovery { |
47 | | - |
| 58 | + private static final Logger LOG = |
| 59 | + LoggerFactory.getLogger(TestClientProtocolForPipelineRecovery.class); |
48 | 60 | @Test public void testGetNewStamp() throws IOException { |
49 | 61 | int numDataNodes = 1; |
50 | 62 | Configuration conf = new HdfsConfiguration(); |
@@ -477,4 +489,128 @@ public void stopSendingPacketDownstream() throws IOException { |
477 | 489 | DataNodeFaultInjector.set(oldDnInjector); |
478 | 490 | } |
479 | 491 | } |
| 492 | + |
| 493 | + // Test to verify that blocks are no longer corrupted after HDFS-4660. |
| 494 | + // Revert HDFS-4660 and the other related ones (HDFS-9220, HDFS-8722), this |
| 495 | + // test would fail. |
| 496 | + // Scenario: Prior to the fix, block get corrupted when the transferBlock |
| 497 | + // happens during pipeline recovery with extra bytes to make up the end of |
| 498 | + // chunk. |
| 499 | + // For verification, Need to fail the pipeline for last datanode when the |
| 500 | + // second datanode have more bytes on disk than already acked bytes. |
| 501 | + // This will enable to transfer extra bytes to the newNode to makeup |
| 502 | + // end-of-chunk during pipeline recovery. This is achieved by the customized |
| 503 | + // DataNodeFaultInjector class in this test. |
| 504 | + // For detailed info, please refer to HDFS-4660 and HDFS-10587. HDFS-9220 |
| 505 | + // fixes an issue in HDFS-4660 patch, and HDFS-8722 is an optimization. |
| 506 | + @Test |
| 507 | + public void testPipelineRecoveryWithTransferBlock() throws Exception { |
| 508 | + final int chunkSize = 512; |
| 509 | + final int oneWriteSize = 5000; |
| 510 | + final int totalSize = 1024 * 1024; |
| 511 | + final int errorInjectionPos = 512; |
| 512 | + Configuration conf = new HdfsConfiguration(); |
| 513 | + // Need 4 datanodes to verify the replaceDatanode during pipeline recovery |
| 514 | + final MiniDFSCluster cluster = |
| 515 | + new MiniDFSCluster.Builder(conf).numDataNodes(4).build(); |
| 516 | + DataNodeFaultInjector old = DataNodeFaultInjector.get(); |
| 517 | + |
| 518 | + try { |
| 519 | + DistributedFileSystem fs = cluster.getFileSystem(); |
| 520 | + Path fileName = new Path("/f"); |
| 521 | + FSDataOutputStream o = fs.create(fileName); |
| 522 | + int count = 0; |
| 523 | + // Flush to get the pipeline created. |
| 524 | + o.writeBytes("hello"); |
| 525 | + o.hflush(); |
| 526 | + DFSOutputStream dfsO = (DFSOutputStream) o.getWrappedStream(); |
| 527 | + final DatanodeInfo[] pipeline = dfsO.getStreamer().getNodes(); |
| 528 | + final String lastDn = pipeline[2].getXferAddr(false); |
| 529 | + final AtomicBoolean failed = new AtomicBoolean(false); |
| 530 | + |
| 531 | + DataNodeFaultInjector.set(new DataNodeFaultInjector() { |
| 532 | + @Override |
| 533 | + public void failPipeline(ReplicaInPipelineInterface replicaInfo, |
| 534 | + String mirror) throws IOException { |
| 535 | + if (!lastDn.equals(mirror)) { |
| 536 | + // Only fail for second DN |
| 537 | + return; |
| 538 | + } |
| 539 | + if (!failed.get() && |
| 540 | + (replicaInfo.getBytesAcked() > errorInjectionPos) && |
| 541 | + (replicaInfo.getBytesAcked() % chunkSize != 0)) { |
| 542 | + int count = 0; |
| 543 | + while (count < 10) { |
| 544 | + // Fail the pipeline (Throw exception) when: |
| 545 | + // 1. bytsAcked is not at chunk boundary (checked in the if |
| 546 | + // statement above) |
| 547 | + // 2. bytesOnDisk is bigger than bytesAcked and at least |
| 548 | + // reaches (or go beyond) the end of the chunk that |
| 549 | + // bytesAcked is in (checked in the if statement below). |
| 550 | + // At this condition, transferBlock that happens during |
| 551 | + // pipeline recovery would transfer extra bytes to make up to the |
| 552 | + // end of the chunk. And this is when the block corruption |
| 553 | + // described in HDFS-4660 would occur. |
| 554 | + if ((replicaInfo.getBytesOnDisk() / chunkSize) - |
| 555 | + (replicaInfo.getBytesAcked() / chunkSize) >= 1) { |
| 556 | + failed.set(true); |
| 557 | + throw new IOException( |
| 558 | + "Failing Pipeline " + replicaInfo.getBytesAcked() + " : " |
| 559 | + + replicaInfo.getBytesOnDisk()); |
| 560 | + } |
| 561 | + try { |
| 562 | + Thread.sleep(200); |
| 563 | + } catch (InterruptedException e) { |
| 564 | + } |
| 565 | + count++; |
| 566 | + } |
| 567 | + } |
| 568 | + } |
| 569 | + }); |
| 570 | + |
| 571 | + Random r = new Random(); |
| 572 | + byte[] b = new byte[oneWriteSize]; |
| 573 | + while (count < totalSize) { |
| 574 | + r.nextBytes(b); |
| 575 | + o.write(b); |
| 576 | + count += oneWriteSize; |
| 577 | + o.hflush(); |
| 578 | + } |
| 579 | + |
| 580 | + assertTrue("Expected a failure in the pipeline", failed.get()); |
| 581 | + DatanodeInfo[] newNodes = dfsO.getStreamer().getNodes(); |
| 582 | + o.close(); |
| 583 | + // Trigger block report to NN |
| 584 | + for (DataNode d: cluster.getDataNodes()) { |
| 585 | + DataNodeTestUtils.triggerBlockReport(d); |
| 586 | + } |
| 587 | + // Read from the replaced datanode to verify the corruption. So shutdown |
| 588 | + // all other nodes in the pipeline. |
| 589 | + List<DatanodeInfo> pipelineList = Arrays.asList(pipeline); |
| 590 | + DatanodeInfo newNode = null; |
| 591 | + for (DatanodeInfo node : newNodes) { |
| 592 | + if (!pipelineList.contains(node)) { |
| 593 | + newNode = node; |
| 594 | + break; |
| 595 | + } |
| 596 | + } |
| 597 | + LOG.info("Number of nodes in pipeline: {} newNode {}", |
| 598 | + newNodes.length, newNode.getName()); |
| 599 | + // shutdown old 2 nodes |
| 600 | + for (int i = 0; i < newNodes.length; i++) { |
| 601 | + if (newNodes[i].getName().equals(newNode.getName())) { |
| 602 | + continue; |
| 603 | + } |
| 604 | + LOG.info("shutdown {}", newNodes[i].getName()); |
| 605 | + cluster.stopDataNode(newNodes[i].getName()); |
| 606 | + } |
| 607 | + |
| 608 | + // Read should be successfull from only the newNode. There should not be |
| 609 | + // any corruption reported. |
| 610 | + DFSTestUtil.readFile(fs, fileName); |
| 611 | + } finally { |
| 612 | + DataNodeFaultInjector.set(old); |
| 613 | + cluster.shutdown(); |
| 614 | + } |
| 615 | + } |
480 | 616 | } |
0 commit comments