Skip to content
Closed
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
Original file line number Diff line number Diff line change
Expand Up @@ -2751,6 +2751,9 @@ public boolean checkBlockReportLease(BlockReportContext context,
return true;
}
DatanodeDescriptor node = datanodeManager.getDatanode(nodeID);
if (node == null) {
throw new UnregisteredNodeException(nodeID, null);
}
final long startTime = Time.monotonicNow();
return blockReportLeaseManager.checkLease(node, startTime,
context.getLeaseId());
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -2187,5 +2187,10 @@ public DatanodeStorageReport[] getDatanodeStorageReport(
}
return reports;
}

@VisibleForTesting
public Map<String, DatanodeDescriptor> getDatanodeMap() {
return datanodeMap;
}
}

Original file line number Diff line number Diff line change
Expand Up @@ -1633,7 +1633,7 @@ public DatanodeCommand blockReport(final DatanodeRegistration nodeReg,
}
}
} catch (UnregisteredNodeException une) {
LOG.debug("Datanode {} is attempting to report but not register yet.",
LOG.warn("Datanode {} is attempting to report but not register yet.",
Copy link
Member

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Isn't this some normal stuff, do we need to change this to warn?

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Thanks @ayushtkn for your review. This log output is generated when the NN is restarted, and it does seem normal. I'll change it back later.

nodeReg);
return RegisterCommand.REGISTER;
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -30,6 +30,7 @@
import org.apache.hadoop.hdfs.server.protocol.FinalizeCommand;
import org.apache.hadoop.hdfs.server.protocol.HeartbeatResponse;
import org.apache.hadoop.hdfs.server.protocol.NamenodeProtocols;
import org.apache.hadoop.hdfs.server.protocol.RegisterCommand;
import org.apache.hadoop.hdfs.server.protocol.SlowDiskReports;
import org.apache.hadoop.hdfs.server.protocol.SlowPeerReports;
import org.apache.hadoop.hdfs.server.protocol.StorageBlockReport;
Expand Down Expand Up @@ -136,6 +137,50 @@ public void testCheckBlockReportLease() throws Exception {
}
}

@Test
public void testCheckBlockReportLeaseWhenDnUnregister() throws Exception {
HdfsConfiguration conf = new HdfsConfiguration();
Random rand = new Random();

try (MiniDFSCluster cluster = new MiniDFSCluster.Builder(conf).build()) {
FSNamesystem fsn = cluster.getNamesystem();
BlockManager blockManager = fsn.getBlockManager();
String poolId = cluster.getNamesystem().getBlockPoolId();
NamenodeProtocols rpcServer = cluster.getNameNodeRpc();

// Remove the unique DataNode to simulate the unregistered situation.
// This is similar to starting NameNode, and DataNodes are not registered yet.
DataNode dn = cluster.getDataNodes().get(0);
blockManager.getDatanodeManager().getDatanodeMap().remove(dn.getDatanodeUuid());

// Trigger BlockReport.
DatanodeRegistration dnRegistration = dn.getDNRegistrationForBP(poolId);
StorageReport[] storages = dn.getFSDataset().getStorageReports(poolId);
ExecutorService pool = Executors.newFixedThreadPool(1);
BlockReportContext brContext = new BlockReportContext(1, 0,
rand.nextLong(), 1);
Future<DatanodeCommand> sendBRFuture = pool.submit(() -> {
// Build every storage with 100 blocks for sending report.
DatanodeStorage[] datanodeStorages
= new DatanodeStorage[storages.length];
for (int i = 0; i < storages.length; i++) {
datanodeStorages[i] = storages[i].getStorage();
}
StorageBlockReport[] reports = createReports(datanodeStorages, 100);

// Send blockReport.
return rpcServer.blockReport(dnRegistration, poolId, reports,
brContext);
});

// When unregistered DataNode triggering the block report, will throw an
// UnregisteredNodeException. After NameNode processing, RegisterCommand
// is returned to the DataNode.
DatanodeCommand datanodeCommand = sendBRFuture.get();
assertTrue(datanodeCommand instanceof RegisterCommand);
}
}

private StorageBlockReport[] createReports(DatanodeStorage[] dnStorages,
int numBlocks) {
int longsPerBlock = 3;
Expand Down