|
31 | 31 | import java.util.ArrayList;
|
32 | 32 | import java.util.List;
|
33 | 33 | import java.util.Random;
|
| 34 | +import java.util.concurrent.TimeoutException; |
| 35 | +import java.util.concurrent.atomic.AtomicInteger; |
34 | 36 |
|
35 | 37 | import org.apache.hadoop.conf.Configuration;
|
36 | 38 | import org.apache.hadoop.fs.FSDataOutputStream;
|
|
41 | 43 | import org.apache.hadoop.hdfs.protocol.DatanodeInfo;
|
42 | 44 | import org.apache.hadoop.hdfs.protocol.DatanodeInfoWithStorage;
|
43 | 45 | import org.apache.hadoop.hdfs.protocol.LocatedBlock;
|
| 46 | +import org.apache.hadoop.hdfs.security.token.block.InvalidBlockTokenException; |
44 | 47 | import org.apache.hadoop.hdfs.server.blockmanagement.DatanodeDescriptor;
|
| 48 | +import org.apache.hadoop.io.IOUtils; |
45 | 49 | import org.apache.hadoop.net.unix.DomainSocket;
|
46 | 50 | import org.apache.hadoop.net.unix.TemporarySocketDirectory;
|
47 | 51 | import org.apache.hadoop.hdfs.client.impl.DfsClientConf;
|
48 | 52 | import org.apache.hadoop.hdfs.client.HdfsClientConfigKeys.Retry;
|
49 | 53 |
|
| 54 | +import org.apache.hadoop.test.GenericTestUtils; |
| 55 | +import org.apache.log4j.Level; |
50 | 56 | import org.junit.Assume;
|
51 | 57 | import org.junit.Test;
|
| 58 | +import org.mockito.Mockito; |
| 59 | +import org.mockito.invocation.InvocationOnMock; |
| 60 | +import org.mockito.stubbing.Answer; |
52 | 61 |
|
53 | 62 | public class TestDFSInputStream {
|
54 | 63 | private void testSkipInner(MiniDFSCluster cluster) throws IOException {
|
@@ -287,4 +296,67 @@ public void testReadWithoutPreferredCachingReplica() throws IOException {
|
287 | 296 | cluster.shutdown();
|
288 | 297 | }
|
289 | 298 | }
|
| 299 | + |
| 300 | + @Test |
| 301 | + public void testCreateBlockReaderWhenInvalidBlockTokenException() throws |
| 302 | + IOException, InterruptedException, TimeoutException { |
| 303 | + GenericTestUtils.setLogLevel(DFSClient.LOG, Level.DEBUG); |
| 304 | + Configuration conf = new HdfsConfiguration(); |
| 305 | + conf.setLong(DFSConfigKeys.DFS_BLOCK_SIZE_KEY, 64 * 1024); |
| 306 | + conf.setInt(HdfsClientConfigKeys.DFS_CLIENT_WRITE_PACKET_SIZE_KEY, 516); |
| 307 | + DFSClientFaultInjector oldFaultInjector = DFSClientFaultInjector.get(); |
| 308 | + FSDataOutputStream out = null; |
| 309 | + try (MiniDFSCluster cluster = new MiniDFSCluster.Builder(conf).numDataNodes(3).build()) { |
| 310 | + cluster.waitActive(); |
| 311 | + DistributedFileSystem fs = cluster.getFileSystem(); |
| 312 | + |
| 313 | + // Create file which only contains one UC block. |
| 314 | + String file = "/testfile"; |
| 315 | + Path path = new Path(file); |
| 316 | + out = fs.create(path, (short) 3); |
| 317 | + int bufferLen = 5120; |
| 318 | + byte[] toWrite = new byte[bufferLen]; |
| 319 | + Random rb = new Random(0); |
| 320 | + rb.nextBytes(toWrite); |
| 321 | + out.write(toWrite, 0, bufferLen); |
| 322 | + |
| 323 | + // Wait for the block length of the file to be 1. |
| 324 | + GenericTestUtils.waitFor(() -> { |
| 325 | + try { |
| 326 | + return fs.getFileBlockLocations(path, 0, bufferLen).length == 1; |
| 327 | + } catch (IOException e) { |
| 328 | + return false; |
| 329 | + } |
| 330 | + }, 100, 10000); |
| 331 | + |
| 332 | + // Set up the InjectionHandler. |
| 333 | + DFSClientFaultInjector.set(Mockito.mock(DFSClientFaultInjector.class)); |
| 334 | + DFSClientFaultInjector injector = DFSClientFaultInjector.get(); |
| 335 | + final AtomicInteger count = new AtomicInteger(0); |
| 336 | + Mockito.doAnswer(new Answer<Void>() { |
| 337 | + @Override |
| 338 | + public Void answer(InvocationOnMock invocation) throws Throwable { |
| 339 | + // Mock access token was invalid when connecting to first datanode |
| 340 | + // throw InvalidBlockTokenException. |
| 341 | + if (count.getAndIncrement() == 0) { |
| 342 | + throw new InvalidBlockTokenException("Mock InvalidBlockTokenException"); |
| 343 | + } |
| 344 | + return null; |
| 345 | + } |
| 346 | + }).when(injector).failCreateBlockReader(); |
| 347 | + |
| 348 | + try (DFSInputStream in = new DFSInputStream(fs.getClient(), file, |
| 349 | + false, null)) { |
| 350 | + int bufLen = 1024; |
| 351 | + byte[] buf = new byte[bufLen]; |
| 352 | + // Seek the offset to 1024 and which should be in the range (0, fileSize). |
| 353 | + in.seek(1024); |
| 354 | + int read = in.read(buf, 0, bufLen); |
| 355 | + assertEquals(1024, read); |
| 356 | + } |
| 357 | + } finally { |
| 358 | + DFSClientFaultInjector.set(oldFaultInjector); |
| 359 | + IOUtils.closeStream(out); |
| 360 | + } |
| 361 | + } |
290 | 362 | }
|
0 commit comments