|
@@ -60,6 +60,7 @@ import org.apache.hadoop.hdfs.server.protocol.StorageBlockReport;
|
|
import org.apache.hadoop.hdfs.server.protocol.StorageReceivedDeletedBlocks;
|
|
import org.apache.hadoop.hdfs.server.protocol.StorageReceivedDeletedBlocks;
|
|
import org.apache.hadoop.hdfs.server.protocol.StorageReport;
|
|
import org.apache.hadoop.hdfs.server.protocol.StorageReport;
|
|
import org.apache.hadoop.io.EnumSetWritable;
|
|
import org.apache.hadoop.io.EnumSetWritable;
|
|
|
|
+import org.apache.hadoop.ipc.RemoteException;
|
|
import org.apache.hadoop.net.DNS;
|
|
import org.apache.hadoop.net.DNS;
|
|
import org.apache.hadoop.net.NetworkTopology;
|
|
import org.apache.hadoop.net.NetworkTopology;
|
|
import org.apache.hadoop.security.Groups;
|
|
import org.apache.hadoop.security.Groups;
|
|
@@ -1153,7 +1154,7 @@ public class NNThroughputBenchmark implements Tool {
|
|
throws IOException {
|
|
throws IOException {
|
|
ExtendedBlock prevBlock = null;
|
|
ExtendedBlock prevBlock = null;
|
|
for(int jdx = 0; jdx < blocksPerFile; jdx++) {
|
|
for(int jdx = 0; jdx < blocksPerFile; jdx++) {
|
|
- LocatedBlock loc = nameNodeProto.addBlock(fileName, clientName,
|
|
|
|
|
|
+ LocatedBlock loc = addBlock(fileName, clientName,
|
|
prevBlock, null, INodeId.GRANDFATHER_INODE_ID, null);
|
|
prevBlock, null, INodeId.GRANDFATHER_INODE_ID, null);
|
|
prevBlock = loc.getBlock();
|
|
prevBlock = loc.getBlock();
|
|
for(DatanodeInfo dnInfo : loc.getLocations()) {
|
|
for(DatanodeInfo dnInfo : loc.getLocations()) {
|
|
@@ -1167,10 +1168,41 @@ public class NNThroughputBenchmark implements Tool {
|
|
nameNodeProto.blockReceivedAndDeleted(datanodes[dnIdx].dnRegistration, loc
|
|
nameNodeProto.blockReceivedAndDeleted(datanodes[dnIdx].dnRegistration, loc
|
|
.getBlock().getBlockPoolId(), report);
|
|
.getBlock().getBlockPoolId(), report);
|
|
}
|
|
}
|
|
|
|
+ // IBRs are asynchronously processed by NameNode. The next
|
|
|
|
+ // ClientProtocol#addBlock() may throw NotReplicatedYetException.
|
|
}
|
|
}
|
|
return prevBlock;
|
|
return prevBlock;
|
|
}
|
|
}
|
|
|
|
|
|
|
|
+ /**
|
|
|
|
+ * Retry ClientProtocol.addBlock() if it throws NotReplicatedYetException.
|
|
|
|
+ * Because addBlock() also commits the previous block,
|
|
|
|
+ * it fails if enough IBRs are not processed by NameNode.
|
|
|
|
+ */
|
|
|
|
+ private LocatedBlock addBlock(String src, String clientName,
|
|
|
|
+ ExtendedBlock previous, DatanodeInfo[] excludeNodes, long fileId,
|
|
|
|
+ String[] favoredNodes) throws IOException {
|
|
|
|
+ for (int i = 0; i < 30; i++) {
|
|
|
|
+ try {
|
|
|
|
+ return nameNodeProto.addBlock(src, clientName,
|
|
|
|
+ previous, excludeNodes, fileId, favoredNodes);
|
|
|
|
+ } catch (NotReplicatedYetException|RemoteException e) {
|
|
|
|
+ if (e instanceof RemoteException) {
|
|
|
|
+ String className = ((RemoteException) e).getClassName();
|
|
|
|
+ if (!className.equals(NotReplicatedYetException.class.getName())) {
|
|
|
|
+ throw e;
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
+ try {
|
|
|
|
+ Thread.sleep(100);
|
|
|
|
+ } catch (InterruptedException ie) {
|
|
|
|
+ LOG.warn("interrupted while retrying addBlock.", ie);
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
+ throw new IOException("failed to add block.");
|
|
|
|
+ }
|
|
|
|
+
|
|
/**
|
|
/**
|
|
* Does not require the argument
|
|
* Does not require the argument
|
|
*/
|
|
*/
|