Skip to content
Merged
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 @@ -97,6 +97,7 @@
import org.apache.hadoop.hdfs.server.namenode.INodesInPath;
import org.apache.hadoop.hdfs.server.namenode.NameNode;
import org.apache.hadoop.hdfs.server.namenode.Namesystem;
import org.apache.hadoop.hdfs.server.namenode.fgl.FSNamesystemLockMode;
import org.apache.hadoop.hdfs.server.namenode.ha.HAContext;
import org.apache.hadoop.hdfs.server.namenode.metrics.NameNodeMetrics;
import org.apache.hadoop.hdfs.server.namenode.sps.StoragePolicySatisfyManager;
Expand Down Expand Up @@ -1895,7 +1896,7 @@ private Block getBlockOnStorage(BlockInfo storedBlock,
*/
public void findAndMarkBlockAsCorrupt(final ExtendedBlock blk,
final DatanodeInfo dn, String storageID, String reason) throws IOException {
assert namesystem.hasWriteLock();
assert namesystem.hasWriteLock(FSNamesystemLockMode.BM);
final Block reportedBlock = blk.getLocalBlock();
final BlockInfo storedBlock = getStoredBlock(reportedBlock);
if (storedBlock == null) {
Expand Down Expand Up @@ -2708,7 +2709,7 @@ void processPendingReconstructions() {
}

public long requestBlockReportLeaseId(DatanodeRegistration nodeReg) {
assert namesystem.hasReadLock();
assert namesystem.hasReadLock(FSNamesystemLockMode.BM);
DatanodeDescriptor node = null;
try {
node = datanodeManager.getDatanode(nodeReg);
Expand All @@ -2730,7 +2731,6 @@ public long requestBlockReportLeaseId(DatanodeRegistration nodeReg) {

public void registerDatanode(DatanodeRegistration nodeReg)
throws IOException {
assert namesystem.hasWriteLock();
datanodeManager.registerDatanode(nodeReg);
bmSafeMode.checkSafeMode();
}
Expand Down Expand Up @@ -2997,7 +2997,7 @@ void removeDNLeaseIfNeeded(DatanodeDescriptor node) {

public void removeBRLeaseIfNeeded(final DatanodeID nodeID,
final BlockReportContext context) throws IOException {
namesystem.writeLock();
namesystem.writeLock(FSNamesystemLockMode.BM);
DatanodeDescriptor node;
try {
node = datanodeManager.getDatanode(nodeID);
Expand All @@ -3015,7 +3015,7 @@ public void removeBRLeaseIfNeeded(final DatanodeID nodeID,
}
}
} finally {
namesystem.writeUnlock("removeBRLeaseIfNeeded");
namesystem.writeUnlock(FSNamesystemLockMode.BM, "removeBRLeaseIfNeeded");
}
}

Expand Down Expand Up @@ -3207,7 +3207,7 @@ public void markBlockReplicasAsCorrupt(Block oldBlock,
BlockInfo block,
long oldGenerationStamp, long oldNumBytes,
DatanodeStorageInfo[] newStorages) throws IOException {
assert namesystem.hasWriteLock();
assert namesystem.hasWriteLock(FSNamesystemLockMode.BM);
BlockToMarkCorrupt b = null;
if (block.getGenerationStamp() != oldGenerationStamp) {
b = new BlockToMarkCorrupt(oldBlock, block, oldGenerationStamp,
Expand Down Expand Up @@ -4415,7 +4415,7 @@ private void removeStoredBlock(DatanodeStorageInfo storageInfo, Block block,
*/
public void removeStoredBlock(BlockInfo storedBlock, DatanodeDescriptor node) {
blockLog.debug("BLOCK* removeStoredBlock: {} from {}", storedBlock, node);
assert (namesystem.hasWriteLock());
assert (namesystem.hasWriteLock(FSNamesystemLockMode.BM));
{
if (storedBlock == null || !blocksMap.removeNode(storedBlock, node)) {
blockLog.debug("BLOCK* removeStoredBlock: {} has already been removed from node {}",
Expand Down Expand Up @@ -4929,7 +4929,7 @@ public int getTotalBlocks() {
}

public void removeBlock(BlockInfo block) {
assert namesystem.hasWriteLock();
assert namesystem.hasWriteLock(FSNamesystemLockMode.BM);
// No need to ACK blocks that are being removed entirely
// from the namespace, since the removal of the associated
// file already removes them from the block map below.
Expand Down Expand Up @@ -4972,7 +4972,7 @@ public void updateLastBlock(BlockInfo lastBlock, ExtendedBlock newBlock) {
/** updates a block in needed reconstruction queue. */
private void updateNeededReconstructions(final BlockInfo block,
final int curReplicasDelta, int expectedReplicasDelta) {
namesystem.writeLock();
namesystem.writeLock(FSNamesystemLockMode.BM);
try {
if (!isPopulatingReplQueues() || !block.isComplete()) {
return;
Expand All @@ -4991,7 +4991,7 @@ private void updateNeededReconstructions(final BlockInfo block,
repl.outOfServiceReplicas(), oldExpectedReplicas);
}
} finally {
namesystem.writeUnlock("updateNeededReconstructions");
namesystem.writeUnlock(FSNamesystemLockMode.BM, "updateNeededReconstructions");
}
}

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -22,6 +22,7 @@

import org.apache.commons.lang3.StringUtils;
import org.apache.hadoop.classification.VisibleForTesting;
import org.apache.hadoop.hdfs.server.namenode.fgl.FSNamesystemLockMode;
import org.apache.hadoop.util.Preconditions;

import org.apache.hadoop.thirdparty.com.google.common.collect.ImmutableList;
Expand Down Expand Up @@ -855,7 +856,7 @@ private void removeDatanode(DatanodeDescriptor nodeInfo) {
*/
private void removeDatanode(DatanodeDescriptor nodeInfo,
boolean removeBlocksFromBlocksMap) {
assert namesystem.hasWriteLock();
assert namesystem.hasWriteLock(FSNamesystemLockMode.BM);
heartbeatManager.removeDatanode(nodeInfo);
if (removeBlocksFromBlocksMap) {
blockManager.removeBlocksAssociatedTo(nodeInfo);
Expand All @@ -874,7 +875,7 @@ private void removeDatanode(DatanodeDescriptor nodeInfo,
*/
public void removeDatanode(final DatanodeID node)
throws UnregisteredNodeException {
namesystem.writeLock();
namesystem.writeLock(FSNamesystemLockMode.BM);
try {
final DatanodeDescriptor descriptor = getDatanode(node);
if (descriptor != null) {
Expand All @@ -884,7 +885,7 @@ public void removeDatanode(final DatanodeID node)
+ node + " does not exist");
}
} finally {
namesystem.writeUnlock("removeDatanode");
namesystem.writeUnlock(FSNamesystemLockMode.BM, "removeDatanode");
}
}

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -45,6 +45,7 @@
import org.apache.hadoop.hdfs.server.common.blockaliasmap.BlockAliasMap;
import org.apache.hadoop.hdfs.server.common.blockaliasmap.impl.TextFileRegionAliasMap;
import org.apache.hadoop.hdfs.server.common.BlockAlias;
import org.apache.hadoop.hdfs.server.namenode.fgl.FSNamesystemLockMode;
import org.apache.hadoop.hdfs.server.protocol.DatanodeStorage;
import org.apache.hadoop.hdfs.server.protocol.DatanodeStorage.State;
import org.apache.hadoop.hdfs.util.RwLock;
Expand Down Expand Up @@ -173,7 +174,7 @@ public LocatedBlockBuilder newLocatedBlocks(int maxValue) {

public void removeDatanode(DatanodeDescriptor dnToRemove) {
if (providedEnabled) {
assert lock.hasWriteLock() : "Not holding write lock";
assert lock.hasWriteLock(FSNamesystemLockMode.BM) : "Not holding write lock";
providedDescriptor.remove(dnToRemove);
// if all datanodes fail, set the block report count to 0
if (providedDescriptor.activeProvidedDatanodes() == 0) {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -17,6 +17,7 @@
*/
package org.apache.hadoop.hdfs.server.namenode;

import org.apache.hadoop.hdfs.server.namenode.fgl.FSNamesystemLockMode;
import org.apache.hadoop.util.Preconditions;
import org.apache.hadoop.HadoopIllegalArgumentException;
import org.apache.hadoop.fs.XAttrSetFlag;
Expand Down Expand Up @@ -104,7 +105,7 @@ static boolean unprotectedRemoveBlock(
*/
static void persistBlocks(
FSDirectory fsd, String path, INodeFile file, boolean logRetryCache) {
assert fsd.getFSNamesystem().hasWriteLock();
assert fsd.getFSNamesystem().hasWriteLock(FSNamesystemLockMode.FS);
Preconditions.checkArgument(file.isUnderConstruction());
fsd.getEditLog().logUpdateBlocks(path, file, logRetryCache);
if(NameNode.stateChangeLog.isDebugEnabled()) {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -1103,7 +1103,7 @@ static void unprotectedUpdateCount(INodesInPath inodesInPath,
*/
public void updateSpaceForCompleteBlock(BlockInfo completeBlk,
INodesInPath inodes) throws IOException {
assert namesystem.hasWriteLock();
assert namesystem.hasWriteLock(FSNamesystemLockMode.GLOBAL);
INodesInPath iip = inodes != null ? inodes :
INodesInPath.fromINode(namesystem.getBlockCollection(completeBlk));
INodeFile fileINode = iip.getLastINode().asFile();
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -1890,11 +1890,11 @@ public void cpUnlock() {


NamespaceInfo getNamespaceInfo() {
readLock();
readLock(FSNamesystemLockMode.FS);
try {
return unprotectedGetNamespaceInfo();
} finally {
readUnlock("getNamespaceInfo");
readUnlock(FSNamesystemLockMode.FS, "getNamespaceInfo");
}
}

Expand Down Expand Up @@ -3901,7 +3901,7 @@ Lease reassignLeaseInternal(Lease lease, String newHolder, INodeFile pendingFile
void commitOrCompleteLastBlock(
final INodeFile fileINode, final INodesInPath iip,
final Block commitBlock) throws IOException {
assert hasWriteLock();
assert hasWriteLock(FSNamesystemLockMode.GLOBAL);
Preconditions.checkArgument(fileINode.isUnderConstruction());
blockManager.commitOrCompleteLastBlock(fileINode, commitBlock, iip);
}
Expand All @@ -3923,7 +3923,7 @@ void addCommittedBlocksToPending(final INodeFile pendingFile) {

void finalizeINodeFileUnderConstruction(String src, INodeFile pendingFile,
int latestSnapshot, boolean allowCommittedBlock) throws IOException {
assert hasWriteLock();
assert hasWriteLock(FSNamesystemLockMode.GLOBAL);

FileUnderConstructionFeature uc = pendingFile.getFileUnderConstructionFeature();
if (uc == null) {
Expand Down Expand Up @@ -3992,7 +3992,8 @@ INodeFile getBlockCollection(BlockInfo b) {

@Override
public INodeFile getBlockCollection(long id) {
assert hasReadLock() : "Accessing INode id = " + id + " without read lock";
assert hasReadLock(FSNamesystemLockMode.FS)
: "Accessing INode id = " + id + " without read lock";
INode inode = getFSDirectory().getInode(id);
return inode == null ? null : inode.asFile();
}
Expand All @@ -4010,7 +4011,7 @@ void commitBlockSynchronization(ExtendedBlock oldBlock,
+ ")");
checkOperation(OperationCategory.WRITE);
final String src;
writeLock();
writeLock(FSNamesystemLockMode.GLOBAL);
boolean copyTruncate = false;
BlockInfo truncatedBlock = null;
try {
Expand Down Expand Up @@ -4145,7 +4146,7 @@ void commitBlockSynchronization(ExtendedBlock oldBlock,
}
blockManager.successfulBlockRecovery(storedBlock);
} finally {
writeUnlock("commitBlockSynchronization");
writeUnlock(FSNamesystemLockMode.GLOBAL, "commitBlockSynchronization");
}
getEditLog().logSync();
if (closeFile) {
Expand Down Expand Up @@ -4406,11 +4407,11 @@ BatchedDirectoryListing getBatchedListing(String[] srcs, byte[] startAfter,
* @see org.apache.hadoop.hdfs.server.datanode.DataNode
*/
void registerDatanode(DatanodeRegistration nodeReg) throws IOException {
writeLock();
writeLock(FSNamesystemLockMode.BM);
try {
blockManager.registerDatanode(nodeReg);
} finally {
writeUnlock("registerDatanode");
writeUnlock(FSNamesystemLockMode.BM, "registerDatanode");
}
}

Expand Down Expand Up @@ -4443,7 +4444,7 @@ HeartbeatResponse handleHeartbeat(DatanodeRegistration nodeReg,
@Nonnull SlowPeerReports slowPeers,
@Nonnull SlowDiskReports slowDisks)
throws IOException {
readLock();
readLock(FSNamesystemLockMode.BM);
try {
//get datanode commands
DatanodeCommand[] cmds = blockManager.getDatanodeManager().handleHeartbeat(
Expand All @@ -4466,7 +4467,7 @@ nodeReg, reports, getBlockPoolId(), cacheCapacity, cacheUsed,
return new HeartbeatResponse(cmds, haState, rollingUpgradeInfo,
blockReportLeaseId, isSlownode);
} finally {
readUnlock("handleHeartbeat");
readUnlock(FSNamesystemLockMode.BM, "handleHeartbeat");
}
}

Expand Down Expand Up @@ -4528,7 +4529,7 @@ void checkAvailableResources() {
* @param file
*/
private void closeFile(String path, INodeFile file) {
assert hasWriteLock();
assert hasWriteLock(FSNamesystemLockMode.FS);
// file is closed
getEditLog().logCloseFile(path, file);
NameNode.stateChangeLog.debug("closeFile: {} with {} blocks is persisted to the file system",
Expand Down Expand Up @@ -5874,7 +5875,7 @@ private long nextBlockId(BlockType blockType) throws IOException {
}

boolean isFileDeleted(INodeFile file) {
assert hasReadLock();
assert hasReadLock(FSNamesystemLockMode.FS);
// Not in the inodeMap or in the snapshot but marked deleted.
if (dir.getInode(file.getId()) == null) {
return true;
Expand Down Expand Up @@ -5952,7 +5953,7 @@ private INodeFile checkUCBlock(ExtendedBlock block,
*/
void reportBadBlocks(LocatedBlock[] blocks) throws IOException {
checkOperation(OperationCategory.WRITE);
writeLock();
writeLock(FSNamesystemLockMode.BM);
try {
checkOperation(OperationCategory.WRITE);
for (int i = 0; i < blocks.length; i++) {
Expand All @@ -5968,7 +5969,7 @@ void reportBadBlocks(LocatedBlock[] blocks) throws IOException {
}
}
} finally {
writeUnlock("reportBadBlocks");
writeUnlock(FSNamesystemLockMode.BM, "reportBadBlocks");
}
}

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -21,6 +21,7 @@
import org.apache.hadoop.fs.FileStatus;
import org.apache.hadoop.fs.FileUtil;
import org.apache.hadoop.hdfs.server.datanode.DataNodeFaultInjector;
import org.apache.hadoop.hdfs.server.namenode.fgl.FSNamesystemLockMode;
import org.apache.hadoop.thirdparty.com.google.common.base.Joiner;
import org.apache.hadoop.thirdparty.com.google.common.collect.ImmutableList;
import org.apache.hadoop.thirdparty.com.google.common.collect.LinkedListMultimap;
Expand Down Expand Up @@ -166,6 +167,10 @@ public void setupMockCluster() throws IOException {
fsn = Mockito.mock(FSNamesystem.class);
Mockito.doReturn(true).when(fsn).hasWriteLock();
Mockito.doReturn(true).when(fsn).hasReadLock();
Mockito.doReturn(true).when(fsn).hasWriteLock(FSNamesystemLockMode.GLOBAL);
Mockito.doReturn(true).when(fsn).hasReadLock(FSNamesystemLockMode.GLOBAL);
Mockito.doReturn(true).when(fsn).hasWriteLock(FSNamesystemLockMode.BM);
Mockito.doReturn(true).when(fsn).hasReadLock(FSNamesystemLockMode.BM);
Mockito.doReturn(true).when(fsn).isRunning();
//Make shouldPopulaeReplQueues return true
HAContext haContext = Mockito.mock(HAContext.class);
Expand Down
Loading