Skip to content

Commit fb05192

Browse files
HDFS-17488. DN can fail IBRs with NPE when a volume is removed (#6759)
1 parent 700b3e4 commit fb05192

File tree

9 files changed

+82
-8
lines changed

9 files changed

+82
-8
lines changed

hadoop-common-project/hadoop-common/src/site/markdown/Metrics.md

Lines changed: 1 addition & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -532,6 +532,7 @@ Each metrics record contains tags such as SessionId and Hostname as additional i
532532
| `NumProcessedCommands` | Num of processed commands of all BPServiceActors |
533533
| `ProcessedCommandsOpNumOps` | Total number of processed commands operations |
534534
| `ProcessedCommandsOpAvgTime` | Average time of processed commands operations in milliseconds |
535+
| `NullStorageBlockReports` | Number of blocks in IBRs that failed due to null storage |
535536

536537
FsVolume
537538
--------

hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/datanode/BPOfferService.java

Lines changed: 7 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -31,6 +31,7 @@
3131
import org.apache.hadoop.hdfs.server.protocol.*;
3232
import org.apache.hadoop.hdfs.server.protocol.BlockECReconstructionCommand.BlockECReconstructionInfo;
3333
import org.apache.hadoop.hdfs.server.protocol.ReceivedDeletedBlockInfo.BlockStatus;
34+
import org.apache.hadoop.thirdparty.com.google.common.base.Joiner;
3435
import org.apache.hadoop.util.Lists;
3536
import org.apache.hadoop.util.Sets;
3637

@@ -324,6 +325,12 @@ private void notifyNamenodeBlock(ExtendedBlock block, BlockStatus status,
324325
final ReceivedDeletedBlockInfo info = new ReceivedDeletedBlockInfo(
325326
block.getLocalBlock(), status, delHint);
326327
final DatanodeStorage storage = dn.getFSDataset().getStorage(storageUuid);
328+
if (storage == null) {
329+
LOG.warn("Trying to add RDBI for null storage UUID {}. Trace: {}", storageUuid,
330+
Joiner.on("\n").join(Thread.currentThread().getStackTrace()));
331+
getDataNode().getMetrics().incrNullStorageBlockReports();
332+
return;
333+
}
327334

328335
for (BPServiceActor actor : bpServices) {
329336
actor.getIbrManager().notifyNamenodeBlock(info, storage,

hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/datanode/DataNode.java

Lines changed: 2 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -4057,7 +4057,8 @@ public void checkDiskError() throws IOException {
40574057
}
40584058
}
40594059

4060-
private void handleVolumeFailures(Set<FsVolumeSpi> unhealthyVolumes) {
4060+
@VisibleForTesting
4061+
public void handleVolumeFailures(Set<FsVolumeSpi> unhealthyVolumes) {
40614062
if (unhealthyVolumes.isEmpty()) {
40624063
LOG.debug("handleVolumeFailures done with empty " +
40634064
"unhealthyVolumes");

hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/datanode/DataNodeFaultInjector.java

Lines changed: 6 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -172,4 +172,10 @@ public void delayDiffRecord() {}
172172
* Just delay getMetaDataInputStream a while.
173173
*/
174174
public void delayGetMetaDataInputStream() {}
175+
176+
/**
177+
* Used in {@link DirectoryScanner#reconcile()} to wait until a storage is removed,
178+
* leaving a stale copy of {@link DirectoryScanner#diffs}.
179+
*/
180+
public void waitUntilStorageRemoved() {}
175181
}

hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/datanode/DirectoryScanner.java

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -466,7 +466,7 @@ void shutdown() {
466466
public void reconcile() throws IOException {
467467
LOG.debug("reconcile start DirectoryScanning");
468468
scan();
469-
469+
DataNodeFaultInjector.get().waitUntilStorageRemoved();
470470
// HDFS-14476: run checkAndUpdate with batch to avoid holding the lock too
471471
// long
472472
int loopCount = 0;

hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/datanode/fsdataset/impl/FsDatasetImpl.java

Lines changed: 7 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -2745,8 +2745,12 @@ public void checkAndUpdate(String bpid, ScanInfo scanInfo)
27452745
curDirScannerNotifyCount = 0;
27462746
lastDirScannerNotifyTime = startTimeMs;
27472747
}
2748-
try (AutoCloseableLock lock = lockManager.writeLock(LockLevel.VOLUME, bpid,
2749-
vol.getStorageID())) {
2748+
String storageUuid = vol.getStorageID();
2749+
try (AutoCloseableLock lock = lockManager.writeLock(LockLevel.VOLUME, bpid, storageUuid)) {
2750+
if (!storageMap.containsKey(storageUuid)) {
2751+
// Storage was already removed
2752+
return;
2753+
}
27502754
memBlockInfo = volumeMap.get(bpid, blockId);
27512755
if (memBlockInfo != null &&
27522756
memBlockInfo.getState() != ReplicaState.FINALIZED) {
@@ -2833,7 +2837,7 @@ public void checkAndUpdate(String bpid, ScanInfo scanInfo)
28332837
maxDirScannerNotifyCount++;
28342838
datanode.notifyNamenodeReceivedBlock(
28352839
new ExtendedBlock(bpid, diskBlockInfo), null,
2836-
vol.getStorageID(), vol.isTransientStorage());
2840+
storageUuid, vol.isTransientStorage());
28372841
}
28382842
if (vol.isTransientStorage()) {
28392843
long lockedBytesReserved =

hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/datanode/metrics/DataNodeMetrics.java

Lines changed: 5 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -185,6 +185,8 @@ public class DataNodeMetrics {
185185
private MutableCounterLong numProcessedCommands;
186186
@Metric("Rate of processed commands of all BPServiceActors")
187187
private MutableRate processedCommandsOp;
188+
@Metric("Number of blocks in IBRs that failed due to null storage")
189+
private MutableCounterLong nullStorageBlockReports;
188190

189191
// FsDatasetImpl local file process metrics.
190192
@Metric private MutableRate createRbwOp;
@@ -812,4 +814,7 @@ public void incrReplaceBlockOpToOtherHost() {
812814
replaceBlockOpToOtherHost.incr();
813815
}
814816

817+
public void incrNullStorageBlockReports() {
818+
nullStorageBlockReports.incr();
819+
}
815820
}

hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/server/datanode/TestBPOfferService.java

Lines changed: 5 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -136,6 +136,7 @@ public class TestBPOfferService {
136136
private FsDatasetSpi<?> mockFSDataset;
137137
private DataSetLockManager dataSetLockManager = new DataSetLockManager();
138138
private boolean isSlownode;
139+
private String mockStorageID;
139140

140141
@Before
141142
public void setupMocks() throws Exception {
@@ -157,6 +158,7 @@ public void setupMocks() throws Exception {
157158
// Set up a simulated dataset with our fake BP
158159
mockFSDataset = Mockito.spy(new SimulatedFSDataset(null, conf));
159160
mockFSDataset.addBlockPool(FAKE_BPID, conf);
161+
mockStorageID = ((SimulatedFSDataset) mockFSDataset).getStorages().get(0).getStorageUuid();
160162

161163
// Wire the dataset to the DN.
162164
Mockito.doReturn(mockFSDataset).when(mockDn).getFSDataset();
@@ -289,7 +291,7 @@ public void testBasicFunctionality() throws Exception {
289291
waitForBlockReport(mockNN2);
290292

291293
// When we receive a block, it should report it to both NNs
292-
bpos.notifyNamenodeReceivedBlock(FAKE_BLOCK, null, "", false);
294+
bpos.notifyNamenodeReceivedBlock(FAKE_BLOCK, null, mockStorageID, false);
293295

294296
ReceivedDeletedBlockInfo[] ret = waitForBlockReceived(FAKE_BLOCK, mockNN1);
295297
assertEquals(1, ret.length);
@@ -1099,7 +1101,7 @@ public void testRefreshNameNodes() throws Exception {
10991101
waitForBlockReport(mockNN2);
11001102

11011103
// When we receive a block, it should report it to both NNs
1102-
bpos.notifyNamenodeReceivedBlock(FAKE_BLOCK, null, "", false);
1104+
bpos.notifyNamenodeReceivedBlock(FAKE_BLOCK, null, mockStorageID, false);
11031105

11041106
ReceivedDeletedBlockInfo[] ret = waitForBlockReceived(FAKE_BLOCK,
11051107
mockNN1);
@@ -1140,7 +1142,7 @@ public void testRefreshNameNodes() throws Exception {
11401142
Mockito.verify(mockNN3).registerDatanode(Mockito.any());
11411143

11421144
// When we receive a block, it should report it to both NNs
1143-
bpos.notifyNamenodeReceivedBlock(FAKE_BLOCK, null, "", false);
1145+
bpos.notifyNamenodeReceivedBlock(FAKE_BLOCK, null, mockStorageID, false);
11441146

11451147
// veridfy new NN recieved block report
11461148
ret = waitForBlockReceived(FAKE_BLOCK, mockNN3);

hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/server/datanode/TestDirectoryScanner.java

Lines changed: 48 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -37,9 +37,11 @@
3737
import java.nio.channels.FileChannel;
3838
import java.util.ArrayList;
3939
import java.util.Collection;
40+
import java.util.HashSet;
4041
import java.util.Iterator;
4142
import java.util.List;
4243
import java.util.Random;
44+
import java.util.Set;
4345
import java.util.concurrent.ExecutorService;
4446
import java.util.concurrent.Executors;
4547
import java.util.concurrent.Future;
@@ -1420,4 +1422,50 @@ private void writeFile(FileSystem fs, int numFiles) throws IOException {
14201422
DFSTestUtil.createFile(fs, filePath, 1, (short) 1, 0);
14211423
}
14221424
}
1425+
1426+
@Test(timeout = 30000)
1427+
public void testNullStorage() throws Exception {
1428+
DataNodeFaultInjector oldInjector = DataNodeFaultInjector.get();
1429+
1430+
Configuration conf = getConfiguration();
1431+
conf.setInt(DFSConfigKeys.DFS_DATANODE_FAILED_VOLUMES_TOLERATED_KEY, 1);
1432+
cluster = new MiniDFSCluster.Builder(conf).build();
1433+
try {
1434+
cluster.waitActive();
1435+
bpid = cluster.getNamesystem().getBlockPoolId();
1436+
fds = DataNodeTestUtils.getFSDataset(cluster.getDataNodes().get(0));
1437+
client = cluster.getFileSystem().getClient();
1438+
conf.setInt(DFSConfigKeys.DFS_DATANODE_DIRECTORYSCAN_THREADS_KEY, 1);
1439+
createFile(GenericTestUtils.getMethodName(), BLOCK_LENGTH, false);
1440+
// Make sure checkAndUpdate will run
1441+
truncateBlockFile();
1442+
1443+
// Mock a volume corruption after DirectoryScanner.scan() but before checkAndUpdate()
1444+
FsVolumeImpl volumeToRemove = fds.getVolumeList().get(0);
1445+
DataNodeFaultInjector injector = new DataNodeFaultInjector() {
1446+
@Override
1447+
public void waitUntilStorageRemoved() {
1448+
Set<FsVolumeSpi> volumesToRemove = new HashSet<>();
1449+
volumesToRemove.add(volumeToRemove);
1450+
cluster.getDataNodes().get(0).handleVolumeFailures(volumesToRemove);
1451+
}
1452+
};
1453+
DataNodeFaultInjector.set(injector);
1454+
1455+
GenericTestUtils.LogCapturer logCapturer =
1456+
GenericTestUtils.LogCapturer.captureLogs(DataNode.LOG);
1457+
scanner = new DirectoryScanner(fds, conf);
1458+
scanner.setRetainDiffs(true);
1459+
scanner.reconcile();
1460+
assertFalse(logCapturer.getOutput()
1461+
.contains("Trying to add RDBI for null storage UUID " + volumeToRemove.getStorageID()));
1462+
} finally {
1463+
if (scanner != null) {
1464+
scanner.shutdown();
1465+
scanner = null;
1466+
}
1467+
cluster.shutdown();
1468+
DataNodeFaultInjector.set(oldInjector);
1469+
}
1470+
}
14231471
}

0 commit comments

Comments
 (0)