|
@@ -34,16 +34,23 @@ import org.apache.commons.logging.Log;
|
|
|
import org.apache.commons.logging.LogFactory;
|
|
|
import org.apache.commons.logging.impl.Log4JLogger;
|
|
|
import org.apache.hadoop.conf.Configuration;
|
|
|
+import org.apache.hadoop.fs.FSDataInputStream;
|
|
|
import org.apache.hadoop.fs.FSDataOutputStream;
|
|
|
import org.apache.hadoop.fs.FileSystem;
|
|
|
import org.apache.hadoop.fs.Path;
|
|
|
+import org.apache.hadoop.fs.permission.FsPermission;
|
|
|
import org.apache.hadoop.ha.HAServiceProtocol.RequestSource;
|
|
|
import org.apache.hadoop.ha.HAServiceProtocol.StateChangeRequestInfo;
|
|
|
+import org.apache.hadoop.hdfs.DFSClient;
|
|
|
+import org.apache.hadoop.hdfs.DFSClientAdapter;
|
|
|
import org.apache.hadoop.hdfs.DFSConfigKeys;
|
|
|
+import org.apache.hadoop.hdfs.DFSOutputStream;
|
|
|
import org.apache.hadoop.hdfs.DFSTestUtil;
|
|
|
import org.apache.hadoop.hdfs.DistributedFileSystem;
|
|
|
import org.apache.hadoop.hdfs.MiniDFSCluster;
|
|
|
import org.apache.hadoop.hdfs.MiniDFSNNTopology;
|
|
|
+import org.apache.hadoop.hdfs.protocol.DatanodeInfo;
|
|
|
+import org.apache.hadoop.hdfs.protocol.ExtendedBlock;
|
|
|
import org.apache.hadoop.hdfs.protocol.HdfsConstants.SafeModeAction;
|
|
|
import org.apache.hadoop.hdfs.server.blockmanagement.BlockManagerTestUtil;
|
|
|
import org.apache.hadoop.hdfs.server.namenode.FSImage;
|
|
@@ -766,4 +773,50 @@ public class TestHASafeMode {
|
|
|
assertFalse("ANN should be out of SafeMode", dfsWithFailOver.isInSafeMode());
|
|
|
}
|
|
|
|
|
|
+ /** Test NN crash and client crash/stuck immediately after block allocation */
|
|
|
+ @Test(timeout = 100000)
|
|
|
+ public void testOpenFileWhenNNAndClientCrashAfterAddBlock() throws Exception {
|
|
|
+ cluster.getConfiguration(0).set(
|
|
|
+ DFSConfigKeys.DFS_NAMENODE_SAFEMODE_THRESHOLD_PCT_KEY, "1.0f");
|
|
|
+ String testData = "testData";
|
|
|
+ // to make sure we write the full block before creating dummy block at NN.
|
|
|
+ cluster.getConfiguration(0).setInt("io.bytes.per.checksum",
|
|
|
+ testData.length());
|
|
|
+ cluster.restartNameNode(0);
|
|
|
+ try {
|
|
|
+ cluster.waitActive();
|
|
|
+ cluster.transitionToActive(0);
|
|
|
+ cluster.transitionToStandby(1);
|
|
|
+ DistributedFileSystem dfs = cluster.getFileSystem(0);
|
|
|
+ String pathString = "/tmp1.txt";
|
|
|
+ Path filePath = new Path(pathString);
|
|
|
+ FSDataOutputStream create = dfs.create(filePath,
|
|
|
+ FsPermission.getDefault(), true, 1024, (short) 3, testData.length(),
|
|
|
+ null);
|
|
|
+ create.write(testData.getBytes());
|
|
|
+ create.hflush();
|
|
|
+ DFSClient client = DFSClientAdapter.getClient(dfs);
|
|
|
+ // add one dummy block at NN, but not write to DataNode
|
|
|
+ ExtendedBlock previousBlock = DFSClientAdapter.getPreviousBlock(client,
|
|
|
+ pathString);
|
|
|
+ DFSClientAdapter.getNamenode(client).addBlock(
|
|
|
+ pathString,
|
|
|
+ client.getClientName(),
|
|
|
+ new ExtendedBlock(previousBlock),
|
|
|
+ new DatanodeInfo[0],
|
|
|
+ DFSClientAdapter.getFileId((DFSOutputStream) create
|
|
|
+ .getWrappedStream()), null);
|
|
|
+ cluster.restartNameNode(0, true);
|
|
|
+ cluster.restartDataNode(0);
|
|
|
+ cluster.transitionToActive(0);
|
|
|
+ // let the block reports be processed.
|
|
|
+ Thread.sleep(2000);
|
|
|
+ FSDataInputStream is = dfs.open(filePath);
|
|
|
+ is.close();
|
|
|
+ dfs.recoverLease(filePath);// initiate recovery
|
|
|
+ assertTrue("Recovery also should be success", dfs.recoverLease(filePath));
|
|
|
+ } finally {
|
|
|
+ cluster.shutdown();
|
|
|
+ }
|
|
|
+ }
|
|
|
}
|