|
@@ -33,6 +33,7 @@ import static org.junit.Assert.fail;
|
|
|
import java.io.IOException;
|
|
|
import java.util.concurrent.ThreadLocalRandom;
|
|
|
|
|
|
+import org.apache.hadoop.hdfs.server.blockmanagement.BlockManagerTestUtil;
|
|
|
import org.apache.hadoop.hdfs.server.datanode.DataNodeFaultInjector;
|
|
|
import org.apache.hadoop.ipc.RemoteException;
|
|
|
import org.apache.hadoop.test.LambdaTestUtils;
|
|
@@ -321,7 +322,8 @@ public class TestFileTruncate {
|
|
|
}
|
|
|
|
|
|
@Test
|
|
|
- public void testSnapshotWithAppendTruncate() throws IOException {
|
|
|
+ public void testSnapshotWithAppendTruncate()
|
|
|
+ throws IOException, InterruptedException {
|
|
|
testSnapshotWithAppendTruncate(0, 1, 2);
|
|
|
testSnapshotWithAppendTruncate(0, 2, 1);
|
|
|
testSnapshotWithAppendTruncate(1, 0, 2);
|
|
@@ -335,7 +337,8 @@ public class TestFileTruncate {
|
|
|
* Delete snapshots in the specified order and verify that
|
|
|
* remaining snapshots are still readable.
|
|
|
*/
|
|
|
- void testSnapshotWithAppendTruncate(int ... deleteOrder) throws IOException {
|
|
|
+ void testSnapshotWithAppendTruncate(int... deleteOrder)
|
|
|
+ throws IOException, InterruptedException {
|
|
|
FSDirectory fsDir = cluster.getNamesystem().getFSDirectory();
|
|
|
fs.mkdirs(parent);
|
|
|
fs.setQuota(parent, 100, 1000);
|
|
@@ -383,16 +386,16 @@ public class TestFileTruncate {
|
|
|
// Truncate to block boundary
|
|
|
int newLength = length[0] + BLOCK_SIZE / 2;
|
|
|
boolean isReady = fs.truncate(src, newLength);
|
|
|
+ BlockManagerTestUtil.waitForMarkedDeleteQueueIsEmpty(
|
|
|
+ cluster.getNamesystem(0).getBlockManager());
|
|
|
assertTrue("Recovery is not expected.", isReady);
|
|
|
assertFileLength(snapshotFiles[2], length[2]);
|
|
|
assertFileLength(snapshotFiles[1], length[1]);
|
|
|
assertFileLength(snapshotFiles[0], length[0]);
|
|
|
assertBlockNotPresent(appendedBlk);
|
|
|
-
|
|
|
// Diskspace consumed should be 16 bytes * 3. [blk 1,2,3 SS:4]
|
|
|
contentSummary = fs.getContentSummary(parent);
|
|
|
assertThat(contentSummary.getSpaceConsumed(), is(48L));
|
|
|
-
|
|
|
// Truncate full block again
|
|
|
newLength = length[0] - BLOCK_SIZE / 2;
|
|
|
isReady = fs.truncate(src, newLength);
|
|
@@ -400,11 +403,9 @@ public class TestFileTruncate {
|
|
|
assertFileLength(snapshotFiles[2], length[2]);
|
|
|
assertFileLength(snapshotFiles[1], length[1]);
|
|
|
assertFileLength(snapshotFiles[0], length[0]);
|
|
|
-
|
|
|
// Diskspace consumed should be 16 bytes * 3. [blk 1,2 SS:3,4]
|
|
|
contentSummary = fs.getContentSummary(parent);
|
|
|
assertThat(contentSummary.getSpaceConsumed(), is(48L));
|
|
|
-
|
|
|
// Truncate half of the last block
|
|
|
newLength -= BLOCK_SIZE / 2;
|
|
|
isReady = fs.truncate(src, newLength);
|
|
@@ -415,15 +416,12 @@ public class TestFileTruncate {
|
|
|
assertFileLength(snapshotFiles[0], length[0]);
|
|
|
Block replacedBlk = getLocatedBlocks(src).getLastLocatedBlock()
|
|
|
.getBlock().getLocalBlock();
|
|
|
-
|
|
|
// Diskspace consumed should be 16 bytes * 3. [blk 1,6 SS:2,3,4]
|
|
|
contentSummary = fs.getContentSummary(parent);
|
|
|
assertThat(contentSummary.getSpaceConsumed(), is(54L));
|
|
|
-
|
|
|
snapshotDir = fs.createSnapshot(parent, ss[3]);
|
|
|
snapshotFiles[3] = new Path(snapshotDir, truncateFile);
|
|
|
length[3] = newLength;
|
|
|
-
|
|
|
// Delete file. Should still be able to read snapshots
|
|
|
int numINodes = fsDir.getInodeMapSize();
|
|
|
isReady = fs.delete(src, false);
|
|
@@ -434,17 +432,15 @@ public class TestFileTruncate {
|
|
|
assertFileLength(snapshotFiles[0], length[0]);
|
|
|
assertEquals("Number of INodes should not change",
|
|
|
numINodes, fsDir.getInodeMapSize());
|
|
|
-
|
|
|
fs.deleteSnapshot(parent, ss[3]);
|
|
|
-
|
|
|
+ BlockManagerTestUtil.waitForMarkedDeleteQueueIsEmpty(
|
|
|
+ cluster.getNamesystem(0).getBlockManager());
|
|
|
assertBlockExists(firstBlk);
|
|
|
assertBlockExists(lastBlk);
|
|
|
assertBlockNotPresent(replacedBlk);
|
|
|
-
|
|
|
// Diskspace consumed should be 16 bytes * 3. [SS:1,2,3,4]
|
|
|
contentSummary = fs.getContentSummary(parent);
|
|
|
assertThat(contentSummary.getSpaceConsumed(), is(48L));
|
|
|
-
|
|
|
// delete snapshots in the specified order
|
|
|
fs.deleteSnapshot(parent, ss[deleteOrder[0]]);
|
|
|
assertFileLength(snapshotFiles[deleteOrder[1]], length[deleteOrder[1]]);
|
|
@@ -453,12 +449,12 @@ public class TestFileTruncate {
|
|
|
assertBlockExists(lastBlk);
|
|
|
assertEquals("Number of INodes should not change",
|
|
|
numINodes, fsDir.getInodeMapSize());
|
|
|
-
|
|
|
// Diskspace consumed should be 16 bytes * 3. [SS:1,2,3,4]
|
|
|
contentSummary = fs.getContentSummary(parent);
|
|
|
assertThat(contentSummary.getSpaceConsumed(), is(48L));
|
|
|
-
|
|
|
fs.deleteSnapshot(parent, ss[deleteOrder[1]]);
|
|
|
+ BlockManagerTestUtil.waitForMarkedDeleteQueueIsEmpty(
|
|
|
+ cluster.getNamesystem(0).getBlockManager());
|
|
|
assertFileLength(snapshotFiles[deleteOrder[2]], length[deleteOrder[2]]);
|
|
|
assertBlockExists(firstBlk);
|
|
|
contentSummary = fs.getContentSummary(parent);
|
|
@@ -472,11 +468,11 @@ public class TestFileTruncate {
|
|
|
}
|
|
|
assertEquals("Number of INodes should not change",
|
|
|
numINodes, fsDir .getInodeMapSize());
|
|
|
-
|
|
|
fs.deleteSnapshot(parent, ss[deleteOrder[2]]);
|
|
|
+ BlockManagerTestUtil.waitForMarkedDeleteQueueIsEmpty(
|
|
|
+ cluster.getNamesystem(0).getBlockManager());
|
|
|
assertBlockNotPresent(firstBlk);
|
|
|
assertBlockNotPresent(lastBlk);
|
|
|
-
|
|
|
// Diskspace consumed should be 0 bytes * 3. []
|
|
|
contentSummary = fs.getContentSummary(parent);
|
|
|
assertThat(contentSummary.getSpaceConsumed(), is(0L));
|
|
@@ -490,7 +486,8 @@ public class TestFileTruncate {
|
|
|
* remaining snapshots are still readable.
|
|
|
*/
|
|
|
@Test
|
|
|
- public void testSnapshotWithTruncates() throws IOException {
|
|
|
+ public void testSnapshotWithTruncates()
|
|
|
+ throws IOException, InterruptedException {
|
|
|
testSnapshotWithTruncates(0, 1, 2);
|
|
|
testSnapshotWithTruncates(0, 2, 1);
|
|
|
testSnapshotWithTruncates(1, 0, 2);
|
|
@@ -499,7 +496,8 @@ public class TestFileTruncate {
|
|
|
testSnapshotWithTruncates(2, 1, 0);
|
|
|
}
|
|
|
|
|
|
- void testSnapshotWithTruncates(int ... deleteOrder) throws IOException {
|
|
|
+ void testSnapshotWithTruncates(int... deleteOrder)
|
|
|
+ throws IOException, InterruptedException {
|
|
|
fs.mkdirs(parent);
|
|
|
fs.setQuota(parent, 100, 1000);
|
|
|
fs.allowSnapshot(parent);
|
|
@@ -546,6 +544,8 @@ public class TestFileTruncate {
|
|
|
assertThat(contentSummary.getSpaceConsumed(), is(42L));
|
|
|
|
|
|
fs.deleteSnapshot(parent, ss[deleteOrder[0]]);
|
|
|
+ BlockManagerTestUtil.waitForMarkedDeleteQueueIsEmpty(
|
|
|
+ cluster.getNamesystem(0).getBlockManager());
|
|
|
assertFileLength(snapshotFiles[deleteOrder[1]], length[deleteOrder[1]]);
|
|
|
assertFileLength(snapshotFiles[deleteOrder[2]], length[deleteOrder[2]]);
|
|
|
assertFileLength(src, length[2]);
|
|
@@ -563,6 +563,8 @@ public class TestFileTruncate {
|
|
|
}
|
|
|
|
|
|
fs.deleteSnapshot(parent, ss[deleteOrder[1]]);
|
|
|
+ BlockManagerTestUtil.waitForMarkedDeleteQueueIsEmpty(
|
|
|
+ cluster.getNamesystem(0).getBlockManager());
|
|
|
assertFileLength(snapshotFiles[deleteOrder[2]], length[deleteOrder[2]]);
|
|
|
assertFileLength(src, length[2]);
|
|
|
assertBlockExists(firstBlk);
|
|
@@ -583,6 +585,8 @@ public class TestFileTruncate {
|
|
|
}
|
|
|
|
|
|
fs.deleteSnapshot(parent, ss[deleteOrder[2]]);
|
|
|
+ BlockManagerTestUtil.waitForMarkedDeleteQueueIsEmpty(
|
|
|
+ cluster.getNamesystem(0).getBlockManager());
|
|
|
assertFileLength(src, length[2]);
|
|
|
assertBlockExists(firstBlk);
|
|
|
|
|
@@ -592,6 +596,8 @@ public class TestFileTruncate {
|
|
|
assertThat(contentSummary.getLength(), is(6L));
|
|
|
|
|
|
fs.delete(src, false);
|
|
|
+ BlockManagerTestUtil.waitForMarkedDeleteQueueIsEmpty(
|
|
|
+ cluster.getNamesystem().getBlockManager());
|
|
|
assertBlockNotPresent(firstBlk);
|
|
|
|
|
|
// Diskspace consumed should be 0 bytes * 3. []
|
|
@@ -1258,7 +1264,8 @@ public class TestFileTruncate {
|
|
|
cluster.getNamesystem().getFSDirectory().getBlockManager()
|
|
|
.getTotalBlocks());
|
|
|
fs.delete(p, true);
|
|
|
-
|
|
|
+ BlockManagerTestUtil.waitForMarkedDeleteQueueIsEmpty(
|
|
|
+ cluster.getNamesystem().getBlockManager());
|
|
|
assertEquals("block num should 0", 0,
|
|
|
cluster.getNamesystem().getFSDirectory().getBlockManager()
|
|
|
.getTotalBlocks());
|