Skip to content

Commit 5a5b244

Browse files
committed
HDFS-7587. Edit log corruption can happen if append fails with a quota violation. Contributed by Jing Zhao.
(cherry picked from commit c7c71cd) Conflicts: hadoop-hdfs-project/hadoop-hdfs/CHANGES.txt
1 parent 6dcc795 commit 5a5b244

File tree

3 files changed

+152
-9
lines changed

3 files changed

+152
-9
lines changed

hadoop-hdfs-project/hadoop-hdfs/CHANGES.txt

Lines changed: 3 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -899,6 +899,9 @@ Release 2.7.0 - UNRELEASED
899899
HDFS-7943. Append cannot handle the last block with length greater than
900900
the preferred block size. (jing9)
901901

902+
HDFS-7587. Edit log corruption can happen if append fails with a quota
903+
violation. (jing9)
904+
902905
BREAKDOWN OF HDFS-7584 SUBTASKS AND RELATED JIRAS
903906

904907
HDFS-7720. Quota by Storage Type API, tools and ClientNameNode

hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/FSNamesystem.java

Lines changed: 59 additions & 6 deletions
Original file line numberDiff line numberDiff line change
@@ -2672,6 +2672,8 @@ LocatedBlock prepareFileForAppend(String src, INodesInPath iip,
26722672
String leaseHolder, String clientMachine, boolean newBlock,
26732673
boolean writeToEditLog, boolean logRetryCache) throws IOException {
26742674
final INodeFile file = iip.getLastINode().asFile();
2675+
final QuotaCounts delta = verifyQuotaForUCBlock(file, iip);
2676+
26752677
file.recordModification(iip.getLatestSnapshotId());
26762678
file.toUnderConstruction(leaseHolder, clientMachine);
26772679

@@ -2681,10 +2683,15 @@ LocatedBlock prepareFileForAppend(String src, INodesInPath iip,
26812683
LocatedBlock ret = null;
26822684
if (!newBlock) {
26832685
ret = blockManager.convertLastBlockToUnderConstruction(file, 0);
2684-
if (ret != null) {
2685-
// update the quota: use the preferred block size for UC block
2686-
final long diff = file.getPreferredBlockSize() - ret.getBlockSize();
2687-
dir.updateSpaceConsumed(iip, 0, diff, file.getBlockReplication());
2686+
if (ret != null && delta != null) {
2687+
Preconditions.checkState(delta.getStorageSpace() >= 0,
2688+
"appending to a block with size larger than the preferred block size");
2689+
dir.writeLock();
2690+
try {
2691+
dir.updateCountNoQuotaCheck(iip, iip.length() - 1, delta);
2692+
} finally {
2693+
dir.writeUnlock();
2694+
}
26882695
}
26892696
} else {
26902697
BlockInfoContiguous lastBlock = file.getLastBlock();
@@ -2700,6 +2707,52 @@ LocatedBlock prepareFileForAppend(String src, INodesInPath iip,
27002707
return ret;
27012708
}
27022709

2710+
/**
2711+
* Verify quota when using the preferred block size for UC block. This is
2712+
* usually used by append and truncate
2713+
* @throws QuotaExceededException when violating the storage quota
2714+
* @return expected quota usage update. null means no change or no need to
2715+
* update quota usage later
2716+
*/
2717+
private QuotaCounts verifyQuotaForUCBlock(INodeFile file, INodesInPath iip)
2718+
throws QuotaExceededException {
2719+
if (!isImageLoaded() || dir.shouldSkipQuotaChecks()) {
2720+
// Do not check quota if editlog is still being processed
2721+
return null;
2722+
}
2723+
if (file.getLastBlock() != null) {
2724+
final QuotaCounts delta = computeQuotaDeltaForUCBlock(file);
2725+
dir.readLock();
2726+
try {
2727+
FSDirectory.verifyQuota(iip, iip.length() - 1, delta, null);
2728+
return delta;
2729+
} finally {
2730+
dir.readUnlock();
2731+
}
2732+
}
2733+
return null;
2734+
}
2735+
2736+
/** Compute quota change for converting a complete block to a UC block */
2737+
private QuotaCounts computeQuotaDeltaForUCBlock(INodeFile file) {
2738+
final QuotaCounts delta = new QuotaCounts.Builder().build();
2739+
final BlockInfoContiguous lastBlock = file.getLastBlock();
2740+
if (lastBlock != null) {
2741+
final long diff = file.getPreferredBlockSize() - lastBlock.getNumBytes();
2742+
final short repl = file.getBlockReplication();
2743+
delta.addStorageSpace(diff * repl);
2744+
final BlockStoragePolicy policy = dir.getBlockStoragePolicySuite()
2745+
.getPolicy(file.getStoragePolicyID());
2746+
List<StorageType> types = policy.chooseStorageTypes(repl);
2747+
for (StorageType t : types) {
2748+
if (t.supportTypeQuota()) {
2749+
delta.addTypeSpace(t, diff);
2750+
}
2751+
}
2752+
}
2753+
return delta;
2754+
}
2755+
27032756
/**
27042757
* Recover lease;
27052758
* Immediately revoke the lease of the current lease holder and start lease
@@ -3106,7 +3159,7 @@ FileState analyzeFileState(String src,
31063159
// doesn't match up with what we think is the last block. There are
31073160
// four possibilities:
31083161
// 1) This is the first block allocation of an append() pipeline
3109-
// which started appending exactly at a block boundary.
3162+
// which started appending exactly at or exceeding the block boundary.
31103163
// In this case, the client isn't passed the previous block,
31113164
// so it makes the allocateBlock() call with previous=null.
31123165
// We can distinguish this since the last block of the file
@@ -3131,7 +3184,7 @@ FileState analyzeFileState(String src,
31313184
BlockInfoContiguous penultimateBlock = pendingFile.getPenultimateBlock();
31323185
if (previous == null &&
31333186
lastBlockInFile != null &&
3134-
lastBlockInFile.getNumBytes() == pendingFile.getPreferredBlockSize() &&
3187+
lastBlockInFile.getNumBytes() >= pendingFile.getPreferredBlockSize() &&
31353188
lastBlockInFile.isComplete()) {
31363189
// Case 1
31373190
if (NameNode.stateChangeLog.isDebugEnabled()) {

hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/server/namenode/TestDiskspaceQuotaUpdate.java

Lines changed: 90 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -26,13 +26,18 @@
2626
import org.apache.hadoop.fs.ContentSummary;
2727
import org.apache.hadoop.fs.FSDataOutputStream;
2828
import org.apache.hadoop.fs.Path;
29+
import org.apache.hadoop.fs.StorageType;
2930
import org.apache.hadoop.hdfs.DFSConfigKeys;
3031
import org.apache.hadoop.hdfs.DFSOutputStream;
3132
import org.apache.hadoop.hdfs.DFSTestUtil;
3233
import org.apache.hadoop.hdfs.DistributedFileSystem;
3334
import org.apache.hadoop.hdfs.MiniDFSCluster;
3435
import org.apache.hadoop.hdfs.client.HdfsDataOutputStream;
36+
import org.apache.hadoop.hdfs.protocol.DSQuotaExceededException;
37+
import org.apache.hadoop.hdfs.protocol.HdfsConstants;
38+
import org.apache.hadoop.ipc.RemoteException;
3539
import org.junit.After;
40+
import org.junit.Assert;
3641
import org.junit.Before;
3742
import org.junit.Test;
3843

@@ -148,12 +153,11 @@ public void testUpdateQuotaForFSync() throws Exception {
148153
final Path foo = new Path("/foo");
149154
final Path bar = new Path(foo, "bar");
150155
DFSTestUtil.createFile(dfs, bar, BLOCKSIZE, REPLICATION, 0L);
151-
dfs.setQuota(foo, Long.MAX_VALUE-1, Long.MAX_VALUE-1);
156+
dfs.setQuota(foo, Long.MAX_VALUE - 1, Long.MAX_VALUE - 1);
152157

153158
FSDataOutputStream out = dfs.append(bar);
154159
out.write(new byte[BLOCKSIZE / 4]);
155-
((DFSOutputStream) out.getWrappedStream()).hsync(EnumSet
156-
.of(HdfsDataOutputStream.SyncFlag.UPDATE_LENGTH));
160+
((DFSOutputStream) out.getWrappedStream()).hsync(EnumSet.of(HdfsDataOutputStream.SyncFlag.UPDATE_LENGTH));
157161

158162
INodeDirectory fooNode = fsdir.getINode4Write(foo.toString()).asDirectory();
159163
QuotaCounts quota = fooNode.getDirectoryWithQuotaFeature()
@@ -182,4 +186,87 @@ public void testUpdateQuotaForFSync() throws Exception {
182186
assertEquals(2, ns); // foo and bar
183187
assertEquals((BLOCKSIZE * 2 + BLOCKSIZE / 2) * REPLICATION, ds);
184188
}
189+
190+
/**
191+
* Test append over storage quota does not mark file as UC or create lease
192+
*/
193+
@Test (timeout=60000)
194+
public void testAppendOverStorageQuota() throws Exception {
195+
final Path dir = new Path("/TestAppendOverQuota");
196+
final Path file = new Path(dir, "file");
197+
198+
// create partial block file
199+
dfs.mkdirs(dir);
200+
DFSTestUtil.createFile(dfs, file, BLOCKSIZE/2, REPLICATION, seed);
201+
202+
// lower quota to cause exception when appending to partial block
203+
dfs.setQuota(dir, Long.MAX_VALUE - 1, 1);
204+
final INodeDirectory dirNode = fsdir.getINode4Write(dir.toString())
205+
.asDirectory();
206+
final long spaceUsed = dirNode.getDirectoryWithQuotaFeature()
207+
.getSpaceConsumed().getStorageSpace();
208+
try {
209+
DFSTestUtil.appendFile(dfs, file, BLOCKSIZE);
210+
Assert.fail("append didn't fail");
211+
} catch (DSQuotaExceededException e) {
212+
// ignore
213+
}
214+
215+
// check that the file exists, isn't UC, and has no dangling lease
216+
INodeFile inode = fsdir.getINode(file.toString()).asFile();
217+
Assert.assertNotNull(inode);
218+
Assert.assertFalse("should not be UC", inode.isUnderConstruction());
219+
Assert.assertNull("should not have a lease", cluster.getNamesystem()
220+
.getLeaseManager().getLeaseByPath(file.toString()));
221+
// make sure the quota usage is unchanged
222+
final long newSpaceUsed = dirNode.getDirectoryWithQuotaFeature()
223+
.getSpaceConsumed().getStorageSpace();
224+
assertEquals(spaceUsed, newSpaceUsed);
225+
// make sure edits aren't corrupted
226+
dfs.recoverLease(file);
227+
cluster.restartNameNodes();
228+
}
229+
230+
/**
231+
* Test append over a specific type of storage quota does not mark file as
232+
* UC or create a lease
233+
*/
234+
@Test (timeout=60000)
235+
public void testAppendOverTypeQuota() throws Exception {
236+
final Path dir = new Path("/TestAppendOverTypeQuota");
237+
final Path file = new Path(dir, "file");
238+
239+
// create partial block file
240+
dfs.mkdirs(dir);
241+
// set the storage policy on dir
242+
dfs.setStoragePolicy(dir, HdfsConstants.ONESSD_STORAGE_POLICY_NAME);
243+
DFSTestUtil.createFile(dfs, file, BLOCKSIZE/2, REPLICATION, seed);
244+
245+
// set quota of SSD to 1L
246+
dfs.setQuotaByStorageType(dir, StorageType.SSD, 1L);
247+
final INodeDirectory dirNode = fsdir.getINode4Write(dir.toString())
248+
.asDirectory();
249+
final long spaceUsed = dirNode.getDirectoryWithQuotaFeature()
250+
.getSpaceConsumed().getStorageSpace();
251+
try {
252+
DFSTestUtil.appendFile(dfs, file, BLOCKSIZE);
253+
Assert.fail("append didn't fail");
254+
} catch (RemoteException e) {
255+
assertTrue(e.getClassName().contains("QuotaByStorageTypeExceededException"));
256+
}
257+
258+
// check that the file exists, isn't UC, and has no dangling lease
259+
INodeFile inode = fsdir.getINode(file.toString()).asFile();
260+
Assert.assertNotNull(inode);
261+
Assert.assertFalse("should not be UC", inode.isUnderConstruction());
262+
Assert.assertNull("should not have a lease", cluster.getNamesystem()
263+
.getLeaseManager().getLeaseByPath(file.toString()));
264+
// make sure the quota usage is unchanged
265+
final long newSpaceUsed = dirNode.getDirectoryWithQuotaFeature()
266+
.getSpaceConsumed().getStorageSpace();
267+
assertEquals(spaceUsed, newSpaceUsed);
268+
// make sure edits aren't corrupted
269+
dfs.recoverLease(file);
270+
cluster.restartNameNodes();
271+
}
185272
}

0 commit comments

Comments
 (0)