HDFS-6922. Add LazyPersist flag to INodeFile, save it in FsImage and edit logs. (Arpit Agarwal)

Conflicts:
	hadoop-hdfs-project/hadoop-hdfs/CHANGES-HDFS-6581.txt
	hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/FSDirectory.java
	hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/FSEditLogLoader.java
	hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/FSImageFormat.java
	hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/FSImageFormatPBINode.java
	hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/FSImageSerialization.java
	hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/FSNamesystem.java
	hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/INodeFile.java
	hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/INodeFileAttributes.java
	hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/NameNodeLayoutVersion.java
	hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/snapshot/FSImageFormatPBSnapshot.java
	hadoop-hdfs-project/hadoop-hdfs/src/main/proto/fsimage.proto
	hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/server/namenode/CreateEditsLog.java
	hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/server/namenode/TestEditLog.java
	hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/server/namenode/TestFSPermissionChecker.java
	hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/server/namenode/TestINodeFile.java
This commit is contained in:
arp 2014-08-27 15:12:19 -07:00 committed by Jitendra Pandey
parent b1d085e7e3
commit 183f4a4dfb
20 changed files with 96 additions and 28 deletions

View File

@ -54,6 +54,12 @@ public interface BlockCollection {
*/ */
public long getPreferredBlockSize(); public long getPreferredBlockSize();
/**
* Return true if the file was created with {@Link CreateFlag#LAZY_PERSIST}.
* @return
*/
public boolean getLazyPersistFlag();
/** /**
* Get block replication for the collection * Get block replication for the collection
* @return block replication value * @return block replication value

View File

@ -299,13 +299,14 @@ public class FSDirectory implements Closeable {
*/ */
INodeFile addFile(String path, PermissionStatus permissions, INodeFile addFile(String path, PermissionStatus permissions,
short replication, long preferredBlockSize, short replication, long preferredBlockSize,
boolean isLazyPersist,
String clientName, String clientMachine) String clientName, String clientMachine)
throws FileAlreadyExistsException, QuotaExceededException, throws FileAlreadyExistsException, QuotaExceededException,
UnresolvedLinkException, SnapshotAccessControlException, AclException { UnresolvedLinkException, SnapshotAccessControlException, AclException {
long modTime = now(); long modTime = now();
INodeFile newNode = newINodeFile(namesystem.allocateNewInodeId(), INodeFile newNode = newINodeFile(namesystem.allocateNewInodeId(),
permissions, modTime, modTime, replication, preferredBlockSize); permissions, modTime, modTime, replication, preferredBlockSize, isLazyPersist);
newNode.toUnderConstruction(clientName, clientMachine); newNode.toUnderConstruction(clientName, clientMachine);
boolean added = false; boolean added = false;
@ -335,6 +336,7 @@ public class FSDirectory implements Closeable {
long modificationTime, long modificationTime,
long atime, long atime,
long preferredBlockSize, long preferredBlockSize,
boolean isLazyPersist,
boolean underConstruction, boolean underConstruction,
String clientName, String clientName,
String clientMachine, String clientMachine,
@ -343,12 +345,12 @@ public class FSDirectory implements Closeable {
assert hasWriteLock(); assert hasWriteLock();
if (underConstruction) { if (underConstruction) {
newNode = newINodeFile(id, permissions, modificationTime, newNode = newINodeFile(id, permissions, modificationTime,
modificationTime, replication, preferredBlockSize, storagePolicyId); modificationTime, replication, preferredBlockSize, storagePolicyId, isLazyPersist);
newNode.toUnderConstruction(clientName, clientMachine); newNode.toUnderConstruction(clientName, clientMachine);
} else { } else {
newNode = newINodeFile(id, permissions, modificationTime, atime, newNode = newINodeFile(id, permissions, modificationTime, atime,
replication, preferredBlockSize, storagePolicyId); replication, preferredBlockSize, storagePolicyId, isLazyPersist);
} }
try { try {
@ -2408,6 +2410,7 @@ public class FSDirectory implements Closeable {
final FileEncryptionInfo feInfo = isRawPath ? null : final FileEncryptionInfo feInfo = isRawPath ? null :
getFileEncryptionInfo(node, snapshot, iip); getFileEncryptionInfo(node, snapshot, iip);
boolean isLazyPersist = false;
if (node.isFile()) { if (node.isFile()) {
final INodeFile fileNode = node.asFile(); final INodeFile fileNode = node.asFile();
size = fileNode.computeFileSize(snapshot); size = fileNode.computeFileSize(snapshot);
@ -2415,6 +2418,7 @@ public class FSDirectory implements Closeable {
blocksize = fileNode.getPreferredBlockSize(); blocksize = fileNode.getPreferredBlockSize();
isEncrypted = (feInfo != null) || isEncrypted = (feInfo != null) ||
(isRawPath && isInAnEZ(INodesInPath.fromINode(node))); (isRawPath && isInAnEZ(INodesInPath.fromINode(node)));
isLazyPersist = fileNode.getLazyPersistFlag();
} else { } else {
isEncrypted = isInAnEZ(INodesInPath.fromINode(node)); isEncrypted = isInAnEZ(INodesInPath.fromINode(node));
} }
@ -2427,6 +2431,7 @@ public class FSDirectory implements Closeable {
node.isDirectory(), node.isDirectory(),
replication, replication,
blocksize, blocksize,
isLazyPersist,
node.getModificationTime(snapshot), node.getModificationTime(snapshot),
node.getAccessTime(snapshot), node.getAccessTime(snapshot),
getPermissionForFileStatus(node, snapshot, isEncrypted), getPermissionForFileStatus(node, snapshot, isEncrypted),
@ -2450,6 +2455,7 @@ public class FSDirectory implements Closeable {
long size = 0; // length is zero for directories long size = 0; // length is zero for directories
short replication = 0; short replication = 0;
long blocksize = 0; long blocksize = 0;
boolean isLazyPersist = false;
LocatedBlocks loc = null; LocatedBlocks loc = null;
final boolean isEncrypted; final boolean isEncrypted;
final FileEncryptionInfo feInfo = isRawPath ? null : final FileEncryptionInfo feInfo = isRawPath ? null :
@ -2458,7 +2464,7 @@ public class FSDirectory implements Closeable {
final INodeFile fileNode = node.asFile(); final INodeFile fileNode = node.asFile();
size = fileNode.computeFileSize(snapshot); size = fileNode.computeFileSize(snapshot);
replication = fileNode.getFileReplication(snapshot); replication = fileNode.getFileReplication(snapshot);
blocksize = fileNode.getPreferredBlockSize(); isLazyPersist = fileNode.getLazyPersistFlag();
final boolean inSnapshot = snapshot != Snapshot.CURRENT_STATE_ID; final boolean inSnapshot = snapshot != Snapshot.CURRENT_STATE_ID;
final boolean isUc = !inSnapshot && fileNode.isUnderConstruction(); final boolean isUc = !inSnapshot && fileNode.isUnderConstruction();
@ -2481,7 +2487,7 @@ public class FSDirectory implements Closeable {
HdfsLocatedFileStatus status = HdfsLocatedFileStatus status =
new HdfsLocatedFileStatus(size, node.isDirectory(), replication, new HdfsLocatedFileStatus(size, node.isDirectory(), replication,
blocksize, node.getModificationTime(snapshot), blocksize, isLazyPersist, node.getModificationTime(snapshot),
node.getAccessTime(snapshot), node.getAccessTime(snapshot),
getPermissionForFileStatus(node, snapshot, isEncrypted), getPermissionForFileStatus(node, snapshot, isEncrypted),
node.getUserName(snapshot), node.getGroupName(snapshot), node.getUserName(snapshot), node.getGroupName(snapshot),

View File

@ -714,6 +714,7 @@ public class FSEditLog implements LogsPurgeable {
.setModificationTime(newNode.getModificationTime()) .setModificationTime(newNode.getModificationTime())
.setAccessTime(newNode.getAccessTime()) .setAccessTime(newNode.getAccessTime())
.setBlockSize(newNode.getPreferredBlockSize()) .setBlockSize(newNode.getPreferredBlockSize())
.setLazyPersistFlag(newNode.getLazyPersistFlag())
.setBlocks(newNode.getBlocks()) .setBlocks(newNode.getBlocks())
.setPermissionStatus(permissions) .setPermissionStatus(permissions)
.setClientName(newNode.getFileUnderConstructionFeature().getClientName()) .setClientName(newNode.getFileUnderConstructionFeature().getClientName())
@ -746,6 +747,7 @@ public class FSEditLog implements LogsPurgeable {
.setModificationTime(newNode.getModificationTime()) .setModificationTime(newNode.getModificationTime())
.setAccessTime(newNode.getAccessTime()) .setAccessTime(newNode.getAccessTime())
.setBlockSize(newNode.getPreferredBlockSize()) .setBlockSize(newNode.getPreferredBlockSize())
.setLazyPersistFlag(newNode.getLazyPersistFlag())
.setBlocks(newNode.getBlocks()) .setBlocks(newNode.getBlocks())
.setPermissionStatus(newNode.getPermissionStatus()); .setPermissionStatus(newNode.getPermissionStatus());

View File

@ -366,7 +366,7 @@ public class FSEditLogLoader {
path, addCloseOp.permissions, addCloseOp.aclEntries, path, addCloseOp.permissions, addCloseOp.aclEntries,
addCloseOp.xAttrs, addCloseOp.xAttrs,
replication, addCloseOp.mtime, addCloseOp.atime, replication, addCloseOp.mtime, addCloseOp.atime,
addCloseOp.blockSize, true, addCloseOp.clientName, addCloseOp.blockSize, addCloseOp.isLazyPersist, true, addCloseOp.clientName,
addCloseOp.clientMachine, addCloseOp.storagePolicyId); addCloseOp.clientMachine, addCloseOp.storagePolicyId);
fsNamesys.leaseManager.addLease(addCloseOp.clientName, path); fsNamesys.leaseManager.addLease(addCloseOp.clientName, path);

View File

@ -406,6 +406,7 @@ public abstract class FSEditLogOp {
long mtime; long mtime;
long atime; long atime;
long blockSize; long blockSize;
boolean isLazyPersist;
Block[] blocks; Block[] blocks;
PermissionStatus permissions; PermissionStatus permissions;
List<AclEntry> aclEntries; List<AclEntry> aclEntries;
@ -455,6 +456,11 @@ public abstract class FSEditLogOp {
return (T)this; return (T)this;
} }
<T extends AddCloseOp> T setLazyPersistFlag(boolean isLazyPersist) {
this.isLazyPersist = isLazyPersist;
return (T)this;
}
<T extends AddCloseOp> T setBlocks(Block[] blocks) { <T extends AddCloseOp> T setBlocks(Block[] blocks) {
if (blocks.length > MAX_BLOCKS) { if (blocks.length > MAX_BLOCKS) {
throw new RuntimeException("Can't have more than " + MAX_BLOCKS + throw new RuntimeException("Can't have more than " + MAX_BLOCKS +
@ -512,6 +518,7 @@ public abstract class FSEditLogOp {
FSImageSerialization.writeLong(mtime, out); FSImageSerialization.writeLong(mtime, out);
FSImageSerialization.writeLong(atime, out); FSImageSerialization.writeLong(atime, out);
FSImageSerialization.writeLong(blockSize, out); FSImageSerialization.writeLong(blockSize, out);
FSImageSerialization.writeInt((isLazyPersist ? 1 : 0), out);
new ArrayWritable(Block.class, blocks).write(out); new ArrayWritable(Block.class, blocks).write(out);
permissions.write(out); permissions.write(out);
@ -581,6 +588,13 @@ public abstract class FSEditLogOp {
this.blockSize = readLong(in); this.blockSize = readLong(in);
} }
if (NameNodeLayoutVersion.supports(
NameNodeLayoutVersion.Feature.LAZY_PERSIST_FILES, logVersion)) {
this.isLazyPersist = (FSImageSerialization.readInt(in) != 0);
} else {
this.isLazyPersist = false;
}
this.blocks = readBlocks(in, logVersion); this.blocks = readBlocks(in, logVersion);
this.permissions = PermissionStatus.read(in); this.permissions = PermissionStatus.read(in);
@ -646,6 +660,8 @@ public abstract class FSEditLogOp {
builder.append(atime); builder.append(atime);
builder.append(", blockSize="); builder.append(", blockSize=");
builder.append(blockSize); builder.append(blockSize);
builder.append(", lazyPersist");
builder.append(isLazyPersist);
builder.append(", blocks="); builder.append(", blocks=");
builder.append(Arrays.toString(blocks)); builder.append(Arrays.toString(blocks));
builder.append(", permissions="); builder.append(", permissions=");
@ -686,6 +702,8 @@ public abstract class FSEditLogOp {
Long.toString(atime)); Long.toString(atime));
XMLUtils.addSaxString(contentHandler, "BLOCKSIZE", XMLUtils.addSaxString(contentHandler, "BLOCKSIZE",
Long.toString(blockSize)); Long.toString(blockSize));
XMLUtils.addSaxString(contentHandler, "LAZY_PERSIST",
Boolean.toString(isLazyPersist));
XMLUtils.addSaxString(contentHandler, "CLIENT_NAME", clientName); XMLUtils.addSaxString(contentHandler, "CLIENT_NAME", clientName);
XMLUtils.addSaxString(contentHandler, "CLIENT_MACHINE", clientMachine); XMLUtils.addSaxString(contentHandler, "CLIENT_MACHINE", clientMachine);
XMLUtils.addSaxString(contentHandler, "OVERWRITE", XMLUtils.addSaxString(contentHandler, "OVERWRITE",
@ -711,6 +729,11 @@ public abstract class FSEditLogOp {
this.mtime = Long.parseLong(st.getValue("MTIME")); this.mtime = Long.parseLong(st.getValue("MTIME"));
this.atime = Long.parseLong(st.getValue("ATIME")); this.atime = Long.parseLong(st.getValue("ATIME"));
this.blockSize = Long.parseLong(st.getValue("BLOCKSIZE")); this.blockSize = Long.parseLong(st.getValue("BLOCKSIZE"));
String lazyPersistString = st.getValueOrNull("LAZY_PERSIST");
this.isLazyPersist =
lazyPersistString != null && Boolean.parseBoolean(lazyPersistString);
this.clientName = st.getValue("CLIENT_NAME"); this.clientName = st.getValue("CLIENT_NAME");
this.clientMachine = st.getValue("CLIENT_MACHINE"); this.clientMachine = st.getValue("CLIENT_MACHINE");
this.overwrite = Boolean.parseBoolean(st.getValueOrNull("OVERWRITE")); this.overwrite = Boolean.parseBoolean(st.getValueOrNull("OVERWRITE"));

View File

@ -785,8 +785,11 @@ public class FSImageFormat {
if (counter != null) { if (counter != null) {
counter.increment(); counter.increment();
} }
// Images in the old format will not have the lazyPersist flag so it is
// safe to pass false always.
final INodeFile file = new INodeFile(inodeId, localName, permissions, final INodeFile file = new INodeFile(inodeId, localName, permissions,
modificationTime, atime, blocks, replication, blockSize, (byte)0); modificationTime, atime, blocks, replication, blockSize, (byte)0, false);
if (underConstruction) { if (underConstruction) {
file.toUnderConstruction(clientName, clientMachine); file.toUnderConstruction(clientName, clientMachine);
} }
@ -889,8 +892,10 @@ public class FSImageFormat {
in.readShort()); in.readShort());
final long preferredBlockSize = in.readLong(); final long preferredBlockSize = in.readLong();
// LazyPersist flag will not be present in old image formats and hence
// can be safely set to false always.
return new INodeFileAttributes.SnapshotCopy(name, permissions, null, modificationTime, return new INodeFileAttributes.SnapshotCopy(name, permissions, null, modificationTime,
accessTime, replication, preferredBlockSize, (byte)0, null); accessTime, replication, preferredBlockSize, (byte)0, false null);
} }
public INodeDirectoryAttributes loadINodeDirectoryAttributes(DataInput in) public INodeDirectoryAttributes loadINodeDirectoryAttributes(DataInput in)

View File

@ -290,7 +290,7 @@ public final class FSImageFormatPBINode {
final INodeFile file = new INodeFile(n.getId(), final INodeFile file = new INodeFile(n.getId(),
n.getName().toByteArray(), permissions, f.getModificationTime(), n.getName().toByteArray(), permissions, f.getModificationTime(),
f.getAccessTime(), blocks, replication, f.getPreferredBlockSize(), f.getAccessTime(), blocks, replication, f.getPreferredBlockSize(),
(byte)f.getStoragePolicyID()); (byte)f.getStoragePolicyID(), f.hasIsLazyPersist() ? f.getIsLazyPersist() : false);
if (f.hasAcl()) { if (f.hasAcl()) {
file.addAclFeature(new AclFeature(loadAclEntries(f.getAcl(), file.addAclFeature(new AclFeature(loadAclEntries(f.getAcl(),
@ -400,7 +400,8 @@ public final class FSImageFormatPBINode {
.setPermission(buildPermissionStatus(file, state.getStringMap())) .setPermission(buildPermissionStatus(file, state.getStringMap()))
.setPreferredBlockSize(file.getPreferredBlockSize()) .setPreferredBlockSize(file.getPreferredBlockSize())
.setReplication(file.getFileReplication()) .setReplication(file.getFileReplication())
.setStoragePolicyID(file.getLocalStoragePolicyID()); .setStoragePolicyID(file.getLocalStoragePolicyID())
.setIsLazyPersist(file.getLazyPersistFlag());
AclFeature f = file.getAclFeature(); AclFeature f = file.getAclFeature();
if (f != null) { if (f != null) {

View File

@ -148,14 +148,16 @@ public class FSImageSerialization {
int numLocs = in.readInt(); int numLocs = in.readInt();
assert numLocs == 0 : "Unexpected block locations"; assert numLocs == 0 : "Unexpected block locations";
// Images in the pre-protobuf format will not have the lazyPersist flag,
// so it is safe to pass false always.
INodeFile file = new INodeFile(inodeId, name, perm, modificationTime, INodeFile file = new INodeFile(inodeId, name, perm, modificationTime,
modificationTime, blocks, blockReplication, preferredBlockSize, (byte)0); modificationTime, blocks, blockReplication, preferredBlockSize, (byte)0, false);
file.toUnderConstruction(clientName, clientMachine); file.toUnderConstruction(clientName, clientMachine);
return file; return file;
} }
// Helper function that writes an INodeUnderConstruction // Helper function that writes an INodeUnderConstruction
// into the input stream // into the output stream
// //
static void writeINodeUnderConstruction(DataOutputStream out, INodeFile cons, static void writeINodeUnderConstruction(DataOutputStream out, INodeFile cons,
String path) throws IOException { String path) throws IOException {

View File

@ -2715,7 +2715,7 @@ public class FSNamesystem implements Namesystem, FSClusterStats,
if (parent != null && mkdirsRecursively(parent.toString(), if (parent != null && mkdirsRecursively(parent.toString(),
permissions, true, now())) { permissions, true, now())) {
newNode = dir.addFile(src, permissions, replication, blockSize, newNode = dir.addFile(src, permissions, replication, blockSize,
holder, clientMachine); isLazyPersist, holder, clientMachine);
} }
if (newNode == null) { if (newNode == null) {

View File

@ -80,7 +80,8 @@ public class INodeFile extends INodeWithAdditionalFields
PREFERRED_BLOCK_SIZE(null, 48, 1), PREFERRED_BLOCK_SIZE(null, 48, 1),
REPLICATION(PREFERRED_BLOCK_SIZE.BITS, 12, 1), REPLICATION(PREFERRED_BLOCK_SIZE.BITS, 12, 1),
STORAGE_POLICY_ID(REPLICATION.BITS, BlockStoragePolicySuite.ID_BIT_LENGTH, STORAGE_POLICY_ID(REPLICATION.BITS, BlockStoragePolicySuite.ID_BIT_LENGTH,
0); 0),
LAZY_PERSIST(REPLICATION.BITS, 4, 0);
private final LongBitFormat BITS; private final LongBitFormat BITS;
@ -106,8 +107,12 @@ public class INodeFile extends INodeWithAdditionalFields
h = PREFERRED_BLOCK_SIZE.BITS.combine(preferredBlockSize, h); h = PREFERRED_BLOCK_SIZE.BITS.combine(preferredBlockSize, h);
h = REPLICATION.BITS.combine(replication, h); h = REPLICATION.BITS.combine(replication, h);
h = STORAGE_POLICY_ID.BITS.combine(storagePolicyID, h); h = STORAGE_POLICY_ID.BITS.combine(storagePolicyID, h);
h = LAZY_PERSIST.BITS.combine(isLazyPersist ? 1 : 0, h);
return h; return h;
} }
static boolean getLazyPersistFlag(long header) {
return LAZY_PERSIST.BITS.retrieve(header) == 0 ? false : true;
}
} }
private long header = 0L; private long header = 0L;
@ -116,10 +121,10 @@ public class INodeFile extends INodeWithAdditionalFields
INodeFile(long id, byte[] name, PermissionStatus permissions, long mtime, INodeFile(long id, byte[] name, PermissionStatus permissions, long mtime,
long atime, BlockInfo[] blklist, short replication, long atime, BlockInfo[] blklist, short replication,
long preferredBlockSize, byte storagePolicyID) { long preferredBlockSize, byte storagePolicyID, boolean isLazyPersist) {
super(id, name, permissions, mtime, atime); super(id, name, permissions, mtime, atime);
header = HeaderFormat.toLong(preferredBlockSize, replication, header = HeaderFormat.toLong(preferredBlockSize, replication,
storagePolicyID); storagePolicyID, isLazyPersist);
this.blocks = blklist; this.blocks = blklist;
} }
@ -392,6 +397,9 @@ public class INodeFile extends INodeWithAdditionalFields
recordModification(latestSnapshotId); recordModification(latestSnapshotId);
setStoragePolicyID(storagePolicyId); setStoragePolicyID(storagePolicyId);
} }
public boolean getLazyPersistFlag() {
return HeaderFormat.getLazyPersistFlag(header);
}
@Override @Override
public long getHeaderLong() { public long getHeaderLong() {

View File

@ -21,7 +21,6 @@ import org.apache.hadoop.classification.InterfaceAudience;
import org.apache.hadoop.fs.permission.PermissionStatus; import org.apache.hadoop.fs.permission.PermissionStatus;
import org.apache.hadoop.hdfs.server.namenode.INodeFile.HeaderFormat; import org.apache.hadoop.hdfs.server.namenode.INodeFile.HeaderFormat;
import org.apache.hadoop.hdfs.server.namenode.XAttrFeature; import org.apache.hadoop.hdfs.server.namenode.XAttrFeature;
/** /**
* The attributes of a file. * The attributes of a file.
*/ */
@ -33,6 +32,8 @@ public interface INodeFileAttributes extends INodeAttributes {
/** @return preferred block size in bytes */ /** @return preferred block size in bytes */
public long getPreferredBlockSize(); public long getPreferredBlockSize();
public boolean getLazyPersistFlag();
/** @return the header as a long. */ /** @return the header as a long. */
public long getHeaderLong(); public long getHeaderLong();
@ -48,10 +49,10 @@ public interface INodeFileAttributes extends INodeAttributes {
public SnapshotCopy(byte[] name, PermissionStatus permissions, public SnapshotCopy(byte[] name, PermissionStatus permissions,
AclFeature aclFeature, long modificationTime, long accessTime, AclFeature aclFeature, long modificationTime, long accessTime,
short replication, long preferredBlockSize, byte storagePolicyID, short replication, long preferredBlockSize, byte storagePolicyID,
XAttrFeature xAttrsFeature) { boolean isTransient, XAttrFeature xAttrsFeature) {
super(name, permissions, aclFeature, modificationTime, accessTime, super(name, permissions, aclFeature, modificationTime, accessTime,
xAttrsFeature); xAttrsFeature);
header = HeaderFormat.toLong(preferredBlockSize, replication, storagePolicyID); header = HeaderFormat.toLong(preferredBlockSize, replication, isTransient, storagePolicyID);
} }
public SnapshotCopy(INodeFile file) { public SnapshotCopy(INodeFile file) {
@ -74,6 +75,9 @@ public interface INodeFileAttributes extends INodeAttributes {
return HeaderFormat.getStoragePolicyID(header); return HeaderFormat.getStoragePolicyID(header);
} }
@Override
public boolean getLazyPersistFlag() { return HeaderFormat.getLazyPersistFlag(header); }
@Override @Override
public long getHeaderLong() { public long getHeaderLong() {
return header; return header;

View File

@ -70,6 +70,8 @@ public class NameNodeLayoutVersion {
"creating file with overwrite"), "creating file with overwrite"),
XATTRS_NAMESPACE_EXT(-59, "Increase number of xattr namespaces"), XATTRS_NAMESPACE_EXT(-59, "Increase number of xattr namespaces"),
BLOCK_STORAGE_POLICY(-60, "Block Storage policy"); BLOCK_STORAGE_POLICY(-60, "Block Storage policy");
LAZY_PERSIST_FILES(-61, "Support for optional lazy persistence of "
+ " files with reduced durability guarantees");
private final FeatureInfo info; private final FeatureInfo info;

View File

@ -221,7 +221,10 @@ public class FSImageFormatPBSnapshot {
.toByteArray(), permission, acl, fileInPb.getModificationTime(), .toByteArray(), permission, acl, fileInPb.getModificationTime(),
fileInPb.getAccessTime(), (short) fileInPb.getReplication(), fileInPb.getAccessTime(), (short) fileInPb.getReplication(),
fileInPb.getPreferredBlockSize(), fileInPb.getPreferredBlockSize(),
(byte)fileInPb.getStoragePolicyID(), xAttrs); (byte)fileInPb.getStoragePolicyID(),
fileInPb.hasIsLazyPersist() ? fileInPb.getIsLazyPersist() : false,
xAttrs);
} }
FileDiff diff = new FileDiff(pbf.getSnapshotId(), copy, null, FileDiff diff = new FileDiff(pbf.getSnapshotId(), copy, null,

View File

@ -434,6 +434,7 @@ class FSImageLoader {
f.getPermission(), stringTable); f.getPermission(), stringTable);
map.put("accessTime", f.getAccessTime()); map.put("accessTime", f.getAccessTime());
map.put("blockSize", f.getPreferredBlockSize()); map.put("blockSize", f.getPreferredBlockSize());
map.put("lazyPersist", f.getIsLazyPersist());
map.put("group", p.getGroupName()); map.put("group", p.getGroupName());
map.put("length", getFileSize(f)); map.put("length", getFileSize(f));
map.put("modificationTime", f.getModificationTime()); map.put("modificationTime", f.getModificationTime());

View File

@ -247,6 +247,10 @@ public final class PBImageXmlWriter {
.o("perferredBlockSize", f.getPreferredBlockSize()) .o("perferredBlockSize", f.getPreferredBlockSize())
.o("permission", dumpPermission(f.getPermission())); .o("permission", dumpPermission(f.getPermission()));
if (f.hasIsLazyPersist()) {
o("lazyPersist", f.getIsLazyPersist());
}
if (f.getBlocksCount() > 0) { if (f.getBlocksCount() > 0) {
out.print("<blocks>"); out.print("<blocks>");
for (BlockProto b : f.getBlocksList()) { for (BlockProto b : f.getBlocksList()) {

View File

@ -139,6 +139,7 @@ message INodeSection {
optional AclFeatureProto acl = 8; optional AclFeatureProto acl = 8;
optional XAttrFeatureProto xAttrs = 9; optional XAttrFeatureProto xAttrs = 9;
optional uint32 storagePolicyID = 10; optional uint32 storagePolicyID = 10;
optional bool isLazyPersist = 11 [default = false];
} }
message INodeDirectory { message INodeDirectory {

View File

@ -82,7 +82,7 @@ public class CreateEditsLog {
} }
final INodeFile inode = new INodeFile(inodeId.nextValue(), null, final INodeFile inode = new INodeFile(inodeId.nextValue(), null,
p, 0L, 0L, blocks, replication, blockSize, (byte)0); p, 0L, 0L, blocks, replication, blockSize, (byte)0, false);
inode.toUnderConstruction("", ""); inode.toUnderConstruction("", "");
// Append path to filename with information about blockIDs // Append path to filename with information about blockIDs
@ -97,7 +97,7 @@ public class CreateEditsLog {
editLog.logMkDir(currentDir, dirInode); editLog.logMkDir(currentDir, dirInode);
} }
INodeFile fileUc = new INodeFile(inodeId.nextValue(), null, INodeFile fileUc = new INodeFile(inodeId.nextValue(), null,
p, 0L, 0L, BlockInfo.EMPTY_ARRAY, replication, blockSize, (byte)0); p, 0L, 0L, BlockInfo.EMPTY_ARRAY, replication, blockSize, (byte)0, false);
fileUc.toUnderConstruction("", ""); fileUc.toUnderConstruction("", "");
editLog.logOpenFile(filePath, fileUc, false, false); editLog.logOpenFile(filePath, fileUc, false, false);
editLog.logCloseFile(filePath, inode); editLog.logCloseFile(filePath, inode);

View File

@ -194,7 +194,7 @@ public class TestEditLog {
for (int i = 0; i < numTransactions; i++) { for (int i = 0; i < numTransactions; i++) {
INodeFile inode = new INodeFile(namesystem.allocateNewInodeId(), null, INodeFile inode = new INodeFile(namesystem.allocateNewInodeId(), null,
p, 0L, 0L, BlockInfo.EMPTY_ARRAY, replication, blockSize, (byte)0); p, 0L, 0L, BlockInfo.EMPTY_ARRAY, replication, blockSize, (byte)0, false);
inode.toUnderConstruction("", ""); inode.toUnderConstruction("", "");
editLog.logOpenFile("/filename" + (startIndex + i), inode, false, false); editLog.logOpenFile("/filename" + (startIndex + i), inode, false, false);

View File

@ -432,7 +432,7 @@ public class TestFSPermissionChecker {
FsPermission.createImmutable(perm)); FsPermission.createImmutable(perm));
INodeFile inodeFile = new INodeFile(INodeId.GRANDFATHER_INODE_ID, INodeFile inodeFile = new INodeFile(INodeId.GRANDFATHER_INODE_ID,
name.getBytes("UTF-8"), permStatus, 0L, 0L, null, REPLICATION, name.getBytes("UTF-8"), permStatus, 0L, 0L, null, REPLICATION,
PREFERRED_BLOCK_SIZE, (byte)0); PREFERRED_BLOCK_SIZE, (byte)0, false);
parent.addChild(inodeFile); parent.addChild(inodeFile);
return inodeFile; return inodeFile;
} }

View File

@ -85,7 +85,7 @@ public class TestINodeFile {
INodeFile createINodeFile(short replication, long preferredBlockSize) { INodeFile createINodeFile(short replication, long preferredBlockSize) {
return new INodeFile(INodeId.GRANDFATHER_INODE_ID, null, perm, 0L, 0L, return new INodeFile(INodeId.GRANDFATHER_INODE_ID, null, perm, 0L, 0L,
null, replication, preferredBlockSize, (byte)0); null, replication, preferredBlockSize, (byte)0, false);
} }
private static INodeFile createINodeFile(byte storagePolicyID) { private static INodeFile createINodeFile(byte storagePolicyID) {
@ -286,7 +286,7 @@ public class TestINodeFile {
INodeFile[] iNodes = new INodeFile[nCount]; INodeFile[] iNodes = new INodeFile[nCount];
for (int i = 0; i < nCount; i++) { for (int i = 0; i < nCount; i++) {
iNodes[i] = new INodeFile(i, null, perm, 0L, 0L, null, replication, iNodes[i] = new INodeFile(i, null, perm, 0L, 0L, null, replication,
preferredBlockSize, (byte)0); preferredBlockSize, (byte)0, false);
iNodes[i].setLocalName(DFSUtil.string2Bytes(fileNamePrefix + i)); iNodes[i].setLocalName(DFSUtil.string2Bytes(fileNamePrefix + i));
BlockInfo newblock = new BlockInfo(replication); BlockInfo newblock = new BlockInfo(replication);
iNodes[i].addBlock(newblock); iNodes[i].addBlock(newblock);
@ -344,7 +344,7 @@ public class TestINodeFile {
{//cast from INodeFileUnderConstruction {//cast from INodeFileUnderConstruction
final INode from = new INodeFile( final INode from = new INodeFile(
INodeId.GRANDFATHER_INODE_ID, null, perm, 0L, 0L, null, replication, INodeId.GRANDFATHER_INODE_ID, null, perm, 0L, 0L, null, replication,
1024L, (byte)0); 1024L, (byte)0, false);
from.asFile().toUnderConstruction("client", "machine"); from.asFile().toUnderConstruction("client", "machine");
//cast to INodeFile, should success //cast to INodeFile, should success
@ -1068,7 +1068,7 @@ public class TestINodeFile {
public void testFileUnderConstruction() { public void testFileUnderConstruction() {
replication = 3; replication = 3;
final INodeFile file = new INodeFile(INodeId.GRANDFATHER_INODE_ID, null, final INodeFile file = new INodeFile(INodeId.GRANDFATHER_INODE_ID, null,
perm, 0L, 0L, null, replication, 1024L, (byte)0); perm, 0L, 0L, null, replication, 1024L, (byte)0, false);
assertFalse(file.isUnderConstruction()); assertFalse(file.isUnderConstruction());
final String clientName = "client"; final String clientName = "client";