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

Large diffs are not rendered by default.

Original file line number Diff line number Diff line change
Expand Up @@ -132,7 +132,7 @@ public void testRandomHealthAndDisconnects() throws Exception {
long st = Time.now();
while (Time.now() - st < runFor) {
cluster.getTestContext().checkException();
serverFactory.closeAll(ServerCnxn.DisconnectReason.SERVER_SHUTDOWN);
//serverFactory.closeAll(ServerCnxn.DisconnectReason.SERVER_SHUTDOWN);
Thread.sleep(50);
}
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -5360,4 +5360,14 @@ public void disableSPS() {
public StoragePolicySatisfyManager getSPSManager() {
return spsManager;
}

public void setExculeSlowDataNodesForWriteEnabled(boolean enable) {
placementPolicies.getPolicy(CONTIGUOUS).setExculeSlowDataNodesEnabled(enable);
placementPolicies.getPolicy(STRIPED).setExculeSlowDataNodesEnabled(enable);
}

@VisibleForTesting
public boolean getEnableExculeSlowDataNodesForWrite(BlockType blockType) {
return placementPolicies.getPolicy(blockType).getExculeSlowDataNodesEnabled();
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -261,4 +261,16 @@ public <T> void splitNodesWithRack(
}
}
}

/**
* Updates the value used for excludeSlowNodesEnabled, which is set by
* {@code DFSConfigKeys.DFS_NAMENODE_BLOCKPLACEMENTPOLICY_EXCLUDE_SLOW_NODES_ENABLED_KEY}
* initially.
*
* @param enable true, we will filter out slow nodes
* when choosing targets for blocks, otherwise false not filter.
*/
public abstract void setExculeSlowDataNodesEnabled(boolean enable);

public abstract boolean getExculeSlowDataNodesEnabled();
}
Original file line number Diff line number Diff line change
Expand Up @@ -103,7 +103,7 @@ private String getText() {
protected double considerLoadFactor;
private boolean preferLocalNode;
private boolean dataNodePeerStatsEnabled;
private boolean excludeSlowNodesEnabled;
private volatile boolean excludeSlowNodesEnabled;
protected NetworkTopology clusterMap;
protected Host2NodesMap host2datanodeMap;
private FSClusterStats stats;
Expand Down Expand Up @@ -1359,5 +1359,15 @@ protected Collection<DatanodeStorageInfo> pickupReplicaSet(
void setPreferLocalNode(boolean prefer) {
this.preferLocalNode = prefer;
}

@Override
public void setExculeSlowDataNodesEnabled(boolean enable) {
this.excludeSlowNodesEnabled = enable;
}

@Override
public boolean getExculeSlowDataNodesEnabled() {
return excludeSlowNodesEnabled;
}
}

Original file line number Diff line number Diff line change
Expand Up @@ -189,6 +189,8 @@
import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_BLOCK_PLACEMENT_EC_CLASSNAME_KEY;
import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_NAMENODE_AVOID_SLOW_DATANODE_FOR_READ_KEY;
import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_NAMENODE_AVOID_SLOW_DATANODE_FOR_READ_DEFAULT;
import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_NAMENODE_BLOCKPLACEMENTPOLICY_EXCLUDE_SLOW_NODES_ENABLED_KEY;
import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_NAMENODE_BLOCKPLACEMENTPOLICY_EXCLUDE_SLOW_NODES_ENABLED_DEFAULT;

import static org.apache.hadoop.util.ExitUtil.terminate;
import static org.apache.hadoop.util.ToolRunner.confirmPrompt;
Expand Down Expand Up @@ -331,7 +333,8 @@ public enum OperationCategory {
DFS_BLOCK_REPLICATOR_CLASSNAME_KEY,
DFS_BLOCK_PLACEMENT_EC_CLASSNAME_KEY,
DFS_IMAGE_PARALLEL_LOAD_KEY,
DFS_NAMENODE_AVOID_SLOW_DATANODE_FOR_READ_KEY));
DFS_NAMENODE_AVOID_SLOW_DATANODE_FOR_READ_KEY,
DFS_NAMENODE_BLOCKPLACEMENTPOLICY_EXCLUDE_SLOW_NODES_ENABLED_KEY));

private static final String USAGE = "Usage: hdfs namenode ["
+ StartupOption.BACKUP.getName() + "] | \n\t["
Expand Down Expand Up @@ -2200,7 +2203,8 @@ protected String reconfigurePropertyImpl(String property, String newVal)
return newVal;
} else if (property.equals(DFS_IMAGE_PARALLEL_LOAD_KEY)) {
return reconfigureParallelLoad(newVal);
} else if (property.equals(DFS_NAMENODE_AVOID_SLOW_DATANODE_FOR_READ_KEY)) {
} else if (property.equals(DFS_NAMENODE_AVOID_SLOW_DATANODE_FOR_READ_KEY)
|| (property.equals(DFS_NAMENODE_BLOCKPLACEMENTPOLICY_EXCLUDE_SLOW_NODES_ENABLED_KEY))) {
return reconfigureSlowNodesParameters(datanodeManager, property, newVal);
} else {
throw new ReconfigurationException(property, newVal, getConf().get(
Expand Down Expand Up @@ -2234,7 +2238,7 @@ private String reconfReplicationParameters(final String newVal,
newSetting = bm.getBlocksReplWorkMultiplier();
} else {
throw new IllegalArgumentException("Unexpected property " +
property + "in reconfReplicationParameters");
property + " in reconfReplicationParameters");
}
LOG.info("RECONFIGURE* changed {} to {}", property, newSetting);
return String.valueOf(newSetting);
Expand Down Expand Up @@ -2390,15 +2394,24 @@ String reconfigureParallelLoad(String newVal) {

String reconfigureSlowNodesParameters(final DatanodeManager datanodeManager,
final String property, final String newVal) throws ReconfigurationException {
BlockManager bm = namesystem.getBlockManager();
namesystem.writeLock();
boolean enable;
try {
if (newVal == null) {
enable = DFS_NAMENODE_AVOID_SLOW_DATANODE_FOR_READ_DEFAULT;
if (property.equals(DFS_NAMENODE_AVOID_SLOW_DATANODE_FOR_READ_KEY)) {
enable = (newVal == null ? DFS_NAMENODE_AVOID_SLOW_DATANODE_FOR_READ_DEFAULT :
Boolean.parseBoolean(newVal));
datanodeManager.setAvoidSlowDataNodesForReadEnabled(enable);
} else if (property.equals(
DFS_NAMENODE_BLOCKPLACEMENTPOLICY_EXCLUDE_SLOW_NODES_ENABLED_KEY)) {
enable = (newVal == null ?
DFS_NAMENODE_BLOCKPLACEMENTPOLICY_EXCLUDE_SLOW_NODES_ENABLED_DEFAULT :
Boolean.parseBoolean(newVal));
bm.setExculeSlowDataNodesForWriteEnabled(enable);
} else {
enable = Boolean.parseBoolean(newVal);
throw new IllegalArgumentException("Unexpected property " +
property + " in reconfigureSlowNodesParameters");
}
datanodeManager.setAvoidSlowDataNodesForReadEnabled(enable);
LOG.info("RECONFIGURE* changed {} to {}", property, newVal);
return Boolean.toString(enable);
} catch (IllegalArgumentException e) {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -34,10 +34,12 @@
import org.apache.hadoop.fs.Path;
import org.apache.hadoop.hdfs.MiniDFSCluster;
import org.apache.hadoop.hdfs.protocol.HdfsConstants.StoragePolicySatisfierMode;
import org.apache.hadoop.hdfs.protocol.BlockType;
import org.apache.hadoop.hdfs.DFSConfigKeys;
import org.apache.hadoop.hdfs.DistributedFileSystem;
import org.apache.hadoop.hdfs.HdfsConfiguration;
import org.apache.hadoop.hdfs.server.blockmanagement.DatanodeManager;
import org.apache.hadoop.hdfs.server.blockmanagement.BlockManager;
import org.apache.hadoop.hdfs.server.namenode.sps.StoragePolicySatisfyManager;
import org.apache.hadoop.ipc.RemoteException;
import org.apache.hadoop.test.GenericTestUtils;
Expand All @@ -52,6 +54,7 @@
import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_STORAGE_POLICY_SATISFIER_MODE_DEFAULT;
import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_BLOCK_INVALIDATE_LIMIT_KEY;
import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_NAMENODE_AVOID_SLOW_DATANODE_FOR_READ_KEY;
import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_NAMENODE_BLOCKPLACEMENTPOLICY_EXCLUDE_SLOW_NODES_ENABLED_KEY;
import static org.apache.hadoop.fs.CommonConfigurationKeys.IPC_BACKOFF_ENABLE_DEFAULT;

public class TestNameNodeReconfigure {
Expand Down Expand Up @@ -399,8 +402,8 @@ public void testEnableParallelLoadAfterReconfigured()
public void testEnableSlowNodesParametersAfterReconfigured()
throws ReconfigurationException {
final NameNode nameNode = cluster.getNameNode();
final DatanodeManager datanodeManager = nameNode.namesystem
.getBlockManager().getDatanodeManager();
final BlockManager blockManager = nameNode.namesystem.getBlockManager();
final DatanodeManager datanodeManager = blockManager.getDatanodeManager();

// By default, avoidSlowDataNodesForRead is false.
assertEquals(false, datanodeManager.getEnableAvoidSlowDataNodesForRead());
Expand All @@ -410,6 +413,21 @@ public void testEnableSlowNodesParametersAfterReconfigured()

// After reconfigured, avoidSlowDataNodesForRead is true.
assertEquals(true, datanodeManager.getEnableAvoidSlowDataNodesForRead());

// By default, excludeSlowNodesEnabled is false.
assertEquals(false, blockManager.
getEnableExculeSlowDataNodesForWrite(BlockType.CONTIGUOUS));
assertEquals(false, blockManager.
getEnableExculeSlowDataNodesForWrite(BlockType.STRIPED));

nameNode.reconfigureProperty(
DFS_NAMENODE_BLOCKPLACEMENTPOLICY_EXCLUDE_SLOW_NODES_ENABLED_KEY, Boolean.toString(true));

// After reconfigured, excludeSlowNodesEnabled is true.
assertEquals(true, blockManager.
getEnableExculeSlowDataNodesForWrite(BlockType.CONTIGUOUS));
assertEquals(true, blockManager.
getEnableExculeSlowDataNodesForWrite(BlockType.STRIPED));
}

@After
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -39,6 +39,7 @@
import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_BLOCK_PLACEMENT_EC_CLASSNAME_KEY;
import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_BLOCK_REPLICATOR_CLASSNAME_KEY;
import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_NAMENODE_AVOID_SLOW_DATANODE_FOR_READ_KEY;
import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_NAMENODE_BLOCKPLACEMENTPOLICY_EXCLUDE_SLOW_NODES_ENABLED_KEY;

import org.apache.commons.io.FileUtils;
import org.apache.commons.text.TextStringBuilder;
Expand Down Expand Up @@ -431,13 +432,14 @@ public void testNameNodeGetReconfigurableProperties() throws IOException {
final List<String> outs = Lists.newArrayList();
final List<String> errs = Lists.newArrayList();
getReconfigurableProperties("namenode", address, outs, errs);
assertEquals(14, outs.size());
assertEquals(15, outs.size());
assertEquals(DFS_BLOCK_PLACEMENT_EC_CLASSNAME_KEY, outs.get(1));
assertEquals(DFS_BLOCK_REPLICATOR_CLASSNAME_KEY, outs.get(2));
assertEquals(DFS_HEARTBEAT_INTERVAL_KEY, outs.get(3));
assertEquals(DFS_IMAGE_PARALLEL_LOAD_KEY, outs.get(4));
assertEquals(DFS_NAMENODE_AVOID_SLOW_DATANODE_FOR_READ_KEY, outs.get(5));
assertEquals(DFS_NAMENODE_HEARTBEAT_RECHECK_INTERVAL_KEY, outs.get(6));
assertEquals(DFS_NAMENODE_BLOCKPLACEMENTPOLICY_EXCLUDE_SLOW_NODES_ENABLED_KEY, outs.get(6));
assertEquals(DFS_NAMENODE_HEARTBEAT_RECHECK_INTERVAL_KEY, outs.get(7));
assertEquals(errs.size(), 0);
}

Expand Down