HDFS-11832. Switch leftover logs to slf4j format in BlockManager.java. Contributed by Hui Xu and Chen Liang.

This commit is contained in:
Akira Ajisaka 2017-05-29 17:30:23 +09:00
parent 6c6a7a5962
commit a7f085d6bf
No known key found for this signature in database
GPG Key ID: C1EDBB9CA400FD50
2 changed files with 72 additions and 98 deletions

View File

@ -498,14 +498,13 @@ public class BlockManager implements BlockStatsMXBean {
bmSafeMode = new BlockManagerSafeMode(this, namesystem, haEnabled, conf); bmSafeMode = new BlockManagerSafeMode(this, namesystem, haEnabled, conf);
LOG.info("defaultReplication = " + defaultReplication); LOG.info("defaultReplication = {}", defaultReplication);
LOG.info("maxReplication = " + maxReplication); LOG.info("maxReplication = {}", maxReplication);
LOG.info("minReplication = " + minReplication); LOG.info("minReplication = {}", minReplication);
LOG.info("maxReplicationStreams = " + maxReplicationStreams); LOG.info("maxReplicationStreams = {}", maxReplicationStreams);
LOG.info("redundancyRecheckInterval = " + redundancyRecheckIntervalMs + LOG.info("redundancyRecheckInterval = {}ms", redundancyRecheckIntervalMs);
"ms"); LOG.info("encryptDataTransfer = {}", encryptDataTransfer);
LOG.info("encryptDataTransfer = " + encryptDataTransfer); LOG.info("maxNumBlocksToLog = {}", maxNumBlocksToLog);
LOG.info("maxNumBlocksToLog = " + maxNumBlocksToLog);
} }
private static BlockTokenSecretManager createBlockTokenSecretManager( private static BlockTokenSecretManager createBlockTokenSecretManager(
@ -513,7 +512,8 @@ public class BlockManager implements BlockStatsMXBean {
final boolean isEnabled = conf.getBoolean( final boolean isEnabled = conf.getBoolean(
DFSConfigKeys.DFS_BLOCK_ACCESS_TOKEN_ENABLE_KEY, DFSConfigKeys.DFS_BLOCK_ACCESS_TOKEN_ENABLE_KEY,
DFSConfigKeys.DFS_BLOCK_ACCESS_TOKEN_ENABLE_DEFAULT); DFSConfigKeys.DFS_BLOCK_ACCESS_TOKEN_ENABLE_DEFAULT);
LOG.info(DFSConfigKeys.DFS_BLOCK_ACCESS_TOKEN_ENABLE_KEY + "=" + isEnabled); LOG.info("{} = {}", DFSConfigKeys.DFS_BLOCK_ACCESS_TOKEN_ENABLE_KEY,
isEnabled);
if (!isEnabled) { if (!isEnabled) {
if (UserGroupInformation.isSecurityEnabled()) { if (UserGroupInformation.isSecurityEnabled()) {
@ -534,12 +534,10 @@ public class BlockManager implements BlockStatsMXBean {
DFSConfigKeys.DFS_BLOCK_ACCESS_TOKEN_LIFETIME_DEFAULT); DFSConfigKeys.DFS_BLOCK_ACCESS_TOKEN_LIFETIME_DEFAULT);
final String encryptionAlgorithm = conf.get( final String encryptionAlgorithm = conf.get(
DFSConfigKeys.DFS_DATA_ENCRYPTION_ALGORITHM_KEY); DFSConfigKeys.DFS_DATA_ENCRYPTION_ALGORITHM_KEY);
LOG.info(DFSConfigKeys.DFS_BLOCK_ACCESS_KEY_UPDATE_INTERVAL_KEY LOG.info("{}={} min(s), {}={} min(s), {}={}",
+ "=" + updateMin + " min(s), " DFSConfigKeys.DFS_BLOCK_ACCESS_KEY_UPDATE_INTERVAL_KEY, updateMin,
+ DFSConfigKeys.DFS_BLOCK_ACCESS_TOKEN_LIFETIME_KEY DFSConfigKeys.DFS_BLOCK_ACCESS_TOKEN_LIFETIME_KEY, lifetimeMin,
+ "=" + lifetimeMin + " min(s), " DFSConfigKeys.DFS_DATA_ENCRYPTION_ALGORITHM_KEY, encryptionAlgorithm);
+ DFSConfigKeys.DFS_DATA_ENCRYPTION_ALGORITHM_KEY
+ "=" + encryptionAlgorithm);
String nsId = DFSUtil.getNamenodeNameServiceId(conf); String nsId = DFSUtil.getNamenodeNameServiceId(conf);
boolean isHaEnabled = HAUtil.isHAEnabled(conf, nsId); boolean isHaEnabled = HAUtil.isHAEnabled(conf, nsId);
@ -692,8 +690,8 @@ public class BlockManager implements BlockStatsMXBean {
Collection<DatanodeDescriptor> corruptNodes = Collection<DatanodeDescriptor> corruptNodes =
corruptReplicas.getNodes(block); corruptReplicas.getNodes(block);
if (corruptNodes == null) { if (corruptNodes == null) {
LOG.warn(block.getBlockId() + LOG.warn("{} is corrupt but has no associated node.",
" is corrupt but has no associated node."); block.getBlockId());
continue; continue;
} }
int numNodesToFind = corruptNodes.size(); int numNodesToFind = corruptNodes.size();
@ -1156,9 +1154,9 @@ public class BlockManager implements BlockStatsMXBean {
final int numCorruptNodes = numReplicas.corruptReplicas(); final int numCorruptNodes = numReplicas.corruptReplicas();
final int numCorruptReplicas = corruptReplicas.numCorruptReplicas(blk); final int numCorruptReplicas = corruptReplicas.numCorruptReplicas(blk);
if (numCorruptNodes != numCorruptReplicas) { if (numCorruptNodes != numCorruptReplicas) {
LOG.warn("Inconsistent number of corrupt replicas for " LOG.warn("Inconsistent number of corrupt replicas for {}"
+ blk + " blockMap has " + numCorruptNodes + " blockMap has {} but corrupt replicas map has {}",
+ " but corrupt replicas map has " + numCorruptReplicas); blk, numCorruptNodes, numCorruptReplicas);
} }
final int numNodes = blocksMap.numNodes(blk); final int numNodes = blocksMap.numNodes(blk);
@ -1232,7 +1230,7 @@ public class BlockManager implements BlockStatsMXBean {
Collections.<LocatedBlock> emptyList(), null, false, feInfo, ecPolicy); Collections.<LocatedBlock> emptyList(), null, false, feInfo, ecPolicy);
} else { } else {
if (LOG.isDebugEnabled()) { if (LOG.isDebugEnabled()) {
LOG.debug("blocks = " + java.util.Arrays.asList(blocks)); LOG.debug("blocks = {}", java.util.Arrays.asList(blocks));
} }
final AccessMode mode = needBlockToken? BlockTokenIdentifier.AccessMode.READ: null; final AccessMode mode = needBlockToken? BlockTokenIdentifier.AccessMode.READ: null;
final List<LocatedBlock> locatedblocks = createLocatedBlockList( final List<LocatedBlock> locatedblocks = createLocatedBlockList(
@ -1829,8 +1827,7 @@ public class BlockManager implements BlockStatsMXBean {
numReplicas); numReplicas);
if(srcNodes == null || srcNodes.length == 0) { if(srcNodes == null || srcNodes.length == 0) {
// block can not be reconstructed from any node // block can not be reconstructed from any node
LOG.debug("Block " + block + " cannot be reconstructed " + LOG.debug("Block {} cannot be reconstructed from any node", block);
"from any node");
return null; return null;
} }
@ -2272,11 +2269,9 @@ public class BlockManager implements BlockStatsMXBean {
removeBlock(b); removeBlock(b);
} }
if (trackBlockCounts) { if (trackBlockCounts) {
if (LOG.isDebugEnabled()) {
LOG.debug("Adjusting safe-mode totals for deletion." LOG.debug("Adjusting safe-mode totals for deletion."
+ "decreasing safeBlocks by " + numRemovedSafe + "decreasing safeBlocks by {}, totalBlocks by {}",
+ ", totalBlocks by " + numRemovedComplete); numRemovedSafe, numRemovedComplete);
}
bmSafeMode.adjustBlockTotals(-numRemovedSafe, -numRemovedComplete); bmSafeMode.adjustBlockTotals(-numRemovedSafe, -numRemovedComplete);
} }
} }
@ -2440,18 +2435,14 @@ public class BlockManager implements BlockStatsMXBean {
BlockInfo bi = getStoredBlock(b); BlockInfo bi = getStoredBlock(b);
if (bi == null) { if (bi == null) {
if (LOG.isDebugEnabled()) {
LOG.debug("BLOCK* rescanPostponedMisreplicatedBlocks: " + LOG.debug("BLOCK* rescanPostponedMisreplicatedBlocks: " +
"Postponed mis-replicated block " + b + " no longer found " + "Postponed mis-replicated block {} no longer found " +
"in block map."); "in block map.", b);
}
continue; continue;
} }
MisReplicationResult res = processMisReplicatedBlock(bi); MisReplicationResult res = processMisReplicatedBlock(bi);
if (LOG.isDebugEnabled()) {
LOG.debug("BLOCK* rescanPostponedMisreplicatedBlocks: " + LOG.debug("BLOCK* rescanPostponedMisreplicatedBlocks: " +
"Re-scanned block " + b + ", result is " + res); "Re-scanned block {}, result is {}", b, res);
}
if (res == MisReplicationResult.POSTPONE) { if (res == MisReplicationResult.POSTPONE) {
rescannedMisreplicatedBlocks.add(b); rescannedMisreplicatedBlocks.add(b);
} }
@ -2461,10 +2452,9 @@ public class BlockManager implements BlockStatsMXBean {
rescannedMisreplicatedBlocks.clear(); rescannedMisreplicatedBlocks.clear();
long endSize = postponedMisreplicatedBlocks.size(); long endSize = postponedMisreplicatedBlocks.size();
namesystem.writeUnlock(); namesystem.writeUnlock();
LOG.info("Rescan of postponedMisreplicatedBlocks completed in " + LOG.info("Rescan of postponedMisreplicatedBlocks completed in {}" +
(Time.monotonicNow() - startTime) + " msecs. " + " msecs. {} blocks are left. {} blocks were removed.",
endSize + " blocks are left. " + (Time.monotonicNow() - startTime), endSize, (startSize - endSize));
(startSize - endSize) + " blocks were removed.");
} }
} }
@ -2603,9 +2593,9 @@ public class BlockManager implements BlockStatsMXBean {
ReplicaState reportedState = iblk.getState(); ReplicaState reportedState = iblk.getState();
if (LOG.isDebugEnabled()) { if (LOG.isDebugEnabled()) {
LOG.debug("Initial report of block " + iblk.getBlockName() LOG.debug("Initial report of block {} on {} size {} replicaState = {}",
+ " on " + storageInfo.getDatanodeDescriptor() + " size " + iblk.getBlockName(), storageInfo.getDatanodeDescriptor(),
iblk.getNumBytes() + " replicaState = " + reportedState); iblk.getNumBytes(), reportedState);
} }
if (shouldPostponeBlocksFromFuture && isGenStampInFuture(iblk)) { if (shouldPostponeBlocksFromFuture && isGenStampInFuture(iblk)) {
queueReportedBlock(storageInfo, iblk, reportedState, queueReportedBlock(storageInfo, iblk, reportedState,
@ -2684,11 +2674,8 @@ public class BlockManager implements BlockStatsMXBean {
ReplicaState reportedState = replica.getState(); ReplicaState reportedState = replica.getState();
if (LOG.isDebugEnabled()) { LOG.debug("Reported block {} on {} size {} replicaState = {}",
LOG.debug("Reported block " + replica replica, dn, replica.getNumBytes(), reportedState);
+ " on " + dn + " size " + replica.getNumBytes()
+ " replicaState = " + reportedState);
}
if (shouldPostponeBlocksFromFuture if (shouldPostponeBlocksFromFuture
&& isGenStampInFuture(replica)) { && isGenStampInFuture(replica)) {
@ -2754,9 +2741,7 @@ public class BlockManager implements BlockStatsMXBean {
BlockUCState ucState = storedBlock.getBlockUCState(); BlockUCState ucState = storedBlock.getBlockUCState();
// Block is on the NN // Block is on the NN
if (LOG.isDebugEnabled()) { LOG.debug("In memory blockUCState = {}", ucState);
LOG.debug("In memory blockUCState = " + ucState);
}
// Ignore replicas already scheduled to be removed from the DN // Ignore replicas already scheduled to be removed from the DN
if (invalidateBlocks.contains(dn, replica)) { if (invalidateBlocks.contains(dn, replica)) {
@ -2799,12 +2784,9 @@ public class BlockManager implements BlockStatsMXBean {
ReplicaState reportedState, String reason) { ReplicaState reportedState, String reason) {
assert shouldPostponeBlocksFromFuture; assert shouldPostponeBlocksFromFuture;
if (LOG.isDebugEnabled()) { LOG.debug("Queueing reported block {} in state {}" +
LOG.debug("Queueing reported block " + block + " from datanode {} for later processing because {}.",
" in state " + reportedState + block, reportedState, storageInfo.getDatanodeDescriptor(), reason);
" from datanode " + storageInfo.getDatanodeDescriptor() +
" for later processing because " + reason + ".");
}
pendingDNMessages.enqueueReportedBlock(storageInfo, block, reportedState); pendingDNMessages.enqueueReportedBlock(storageInfo, block, reportedState);
} }
@ -2825,9 +2807,7 @@ public class BlockManager implements BlockStatsMXBean {
private void processQueuedMessages(Iterable<ReportedBlockInfo> rbis) private void processQueuedMessages(Iterable<ReportedBlockInfo> rbis)
throws IOException { throws IOException {
for (ReportedBlockInfo rbi : rbis) { for (ReportedBlockInfo rbi : rbis) {
if (LOG.isDebugEnabled()) { LOG.debug("Processing previouly queued message {}", rbi);
LOG.debug("Processing previouly queued message " + rbi);
}
if (rbi.getReportedState() == null) { if (rbi.getReportedState() == null) {
// This is a DELETE_BLOCK request // This is a DELETE_BLOCK request
DatanodeStorageInfo storageInfo = rbi.getStorageInfo(); DatanodeStorageInfo storageInfo = rbi.getStorageInfo();
@ -2852,8 +2832,8 @@ public class BlockManager implements BlockStatsMXBean {
"block postponement."; "block postponement.";
int count = pendingDNMessages.count(); int count = pendingDNMessages.count();
if (count > 0) { if (count > 0) {
LOG.info("Processing " + count + " messages from DataNodes " + LOG.info("Processing {} messages from DataNodes " +
"that were previously queued during standby state"); "that were previously queued during standby state", count);
} }
processQueuedMessages(pendingDNMessages.takeAll()); processQueuedMessages(pendingDNMessages.takeAll());
assert pendingDNMessages.count() == 0; assert pendingDNMessages.count() == 0;
@ -2935,9 +2915,8 @@ public class BlockManager implements BlockStatsMXBean {
// the block report got a little bit delayed after the pipeline // the block report got a little bit delayed after the pipeline
// closed. So, ignore this report, assuming we will get a // closed. So, ignore this report, assuming we will get a
// FINALIZED replica later. See HDFS-2791 // FINALIZED replica later. See HDFS-2791
LOG.info("Received an RBW replica for " + storedBlock + LOG.info("Received an RBW replica for {} on {}: ignoring it, since "
" on " + dn + ": ignoring it, since it is " + + "it is complete with the same genstamp", storedBlock, dn);
"complete with the same genstamp");
return null; return null;
} else { } else {
return new BlockToMarkCorrupt(new Block(reported), storedBlock, return new BlockToMarkCorrupt(new Block(reported), storedBlock,
@ -2952,7 +2931,7 @@ public class BlockManager implements BlockStatsMXBean {
+ " for block: " + storedBlock + + " for block: " + storedBlock +
" on " + dn + " size " + storedBlock.getNumBytes(); " on " + dn + " size " + storedBlock.getNumBytes();
// log here at WARN level since this is really a broken HDFS invariant // log here at WARN level since this is really a broken HDFS invariant
LOG.warn(msg); LOG.warn("{}", msg);
return new BlockToMarkCorrupt(new Block(reported), storedBlock, msg, return new BlockToMarkCorrupt(new Block(reported), storedBlock, msg,
Reason.INVALID_STATE); Reason.INVALID_STATE);
} }
@ -3131,9 +3110,9 @@ public class BlockManager implements BlockStatsMXBean {
int corruptReplicasCount = corruptReplicas.numCorruptReplicas(storedBlock); int corruptReplicasCount = corruptReplicas.numCorruptReplicas(storedBlock);
int numCorruptNodes = num.corruptReplicas(); int numCorruptNodes = num.corruptReplicas();
if (numCorruptNodes != corruptReplicasCount) { if (numCorruptNodes != corruptReplicasCount) {
LOG.warn("Inconsistent number of corrupt replicas for " + LOG.warn("Inconsistent number of corrupt replicas for {}" +
storedBlock + ". blockMap has " + numCorruptNodes + ". blockMap has {} but corrupt replicas map has {}",
" but corrupt replicas map has " + corruptReplicasCount); storedBlock, numCorruptNodes, corruptReplicasCount);
} }
if ((corruptReplicasCount > 0) && (numLiveReplicas >= fileRedundancy)) { if ((corruptReplicasCount > 0) && (numLiveReplicas >= fileRedundancy)) {
invalidateCorruptReplicas(storedBlock, reportedBlock, num); invalidateCorruptReplicas(storedBlock, reportedBlock, num);
@ -3298,12 +3277,13 @@ public class BlockManager implements BlockStatsMXBean {
/ totalBlocks, 1.0); / totalBlocks, 1.0);
if (!blocksItr.hasNext()) { if (!blocksItr.hasNext()) {
LOG.info("Total number of blocks = " + blocksMap.size()); LOG.info("Total number of blocks = {}", blocksMap.size());
LOG.info("Number of invalid blocks = " + nrInvalid); LOG.info("Number of invalid blocks = {}", nrInvalid);
LOG.info("Number of under-replicated blocks = " + nrUnderReplicated); LOG.info("Number of under-replicated blocks = {}", nrUnderReplicated);
LOG.info("Number of over-replicated blocks = " + nrOverReplicated LOG.info("Number of over-replicated blocks = {}{}", nrOverReplicated,
+ ((nrPostponed > 0) ? (" (" + nrPostponed + " postponed)") : "")); ((nrPostponed > 0) ? (" (" + nrPostponed + " postponed)") : ""));
LOG.info("Number of blocks being written = " + nrUnderConstruction); LOG.info("Number of blocks being written = {}",
nrUnderConstruction);
NameNode.stateChangeLog NameNode.stateChangeLog
.info("STATE* Replication Queue initialization " .info("STATE* Replication Queue initialization "
+ "scan for invalid, over- and under-replicated blocks " + "scan for invalid, over- and under-replicated blocks "
@ -3730,11 +3710,8 @@ public class BlockManager implements BlockStatsMXBean {
final DatanodeDescriptor node = storageInfo.getDatanodeDescriptor(); final DatanodeDescriptor node = storageInfo.getDatanodeDescriptor();
if(LOG.isDebugEnabled()) { LOG.debug("Reported block {} on {} size {} replicaState = {}",
LOG.debug("Reported block " + block block, node, block.getNumBytes(), reportedState);
+ " on " + node + " size " + block.getNumBytes()
+ " replicaState = " + reportedState);
}
if (shouldPostponeBlocksFromFuture && if (shouldPostponeBlocksFromFuture &&
isGenStampInFuture(block)) { isGenStampInFuture(block)) {
@ -3756,9 +3733,7 @@ public class BlockManager implements BlockStatsMXBean {
BlockUCState ucState = storedBlock.getBlockUCState(); BlockUCState ucState = storedBlock.getBlockUCState();
// Block is on the NN // Block is on the NN
if(LOG.isDebugEnabled()) { LOG.debug("In memory blockUCState = {}", ucState);
LOG.debug("In memory blockUCState = " + ucState);
}
// Ignore replicas already scheduled to be removed from the DN // Ignore replicas already scheduled to be removed from the DN
if(invalidateBlocks.contains(node, block)) { if(invalidateBlocks.contains(node, block)) {
@ -4013,8 +3988,8 @@ public class BlockManager implements BlockStatsMXBean {
numExtraRedundancy++; numExtraRedundancy++;
} }
} }
LOG.info("Invalidated " + numExtraRedundancy LOG.info("Invalidated {} extra redundancy blocks on {} after "
+ " extra redundancy blocks on " + srcNode + " after it is in service"); + "it is in service", numExtraRedundancy, srcNode);
} }
/** /**
@ -4175,8 +4150,8 @@ public class BlockManager implements BlockStatsMXBean {
try { try {
DatanodeDescriptor dnDescriptor = datanodeManager.getDatanode(dn); DatanodeDescriptor dnDescriptor = datanodeManager.getDatanode(dn);
if (dnDescriptor == null) { if (dnDescriptor == null) {
LOG.warn("DataNode " + dn + " cannot be found with UUID " + LOG.warn("DataNode {} cannot be found with UUID {}" +
dn.getDatanodeUuid() + ", removing block invalidation work."); ", removing block invalidation work.", dn, dn.getDatanodeUuid());
invalidateBlocks.remove(dn); invalidateBlocks.remove(dn);
return 0; return 0;
} }

View File

@ -70,13 +70,14 @@ class InvalidateBlocks {
} }
private void printBlockDeletionTime(final Logger log) { private void printBlockDeletionTime(final Logger log) {
log.info(DFSConfigKeys.DFS_NAMENODE_STARTUP_DELAY_BLOCK_DELETION_SEC_KEY log.info("{} is set to {}",
+ " is set to " + DFSUtil.durationToString(pendingPeriodInMs)); DFSConfigKeys.DFS_NAMENODE_STARTUP_DELAY_BLOCK_DELETION_SEC_KEY,
DFSUtil.durationToString(pendingPeriodInMs));
SimpleDateFormat sdf = new SimpleDateFormat("yyyy MMM dd HH:mm:ss"); SimpleDateFormat sdf = new SimpleDateFormat("yyyy MMM dd HH:mm:ss");
Calendar calendar = new GregorianCalendar(); Calendar calendar = new GregorianCalendar();
calendar.add(Calendar.SECOND, (int) (this.pendingPeriodInMs / 1000)); calendar.add(Calendar.SECOND, (int) (this.pendingPeriodInMs / 1000));
log.info("The block deletion will start around " log.info("The block deletion will start around {}",
+ sdf.format(calendar.getTime())); sdf.format(calendar.getTime()));
} }
/** @return the number of blocks to be invalidated . */ /** @return the number of blocks to be invalidated . */
@ -173,11 +174,9 @@ class InvalidateBlocks {
synchronized List<Block> invalidateWork(final DatanodeDescriptor dn) { synchronized List<Block> invalidateWork(final DatanodeDescriptor dn) {
final long delay = getInvalidationDelay(); final long delay = getInvalidationDelay();
if (delay > 0) { if (delay > 0) {
if (BlockManager.LOG.isDebugEnabled()) {
BlockManager.LOG BlockManager.LOG
.debug("Block deletion is delayed during NameNode startup. " .debug("Block deletion is delayed during NameNode startup. "
+ "The deletion will start after " + delay + " ms."); + "The deletion will start after {} ms.", delay);
}
return null; return null;
} }
final LightWeightHashSet<Block> set = node2blocks.get(dn); final LightWeightHashSet<Block> set = node2blocks.get(dn);