1 /**
2  * Licensed to the Apache Software Foundation (ASF) under one
3  * or more contributor license agreements.  See the NOTICE file
4  * distributed with this work for additional information
5  * regarding copyright ownership.  The ASF licenses this file
6  * to you under the Apache License, Version 2.0 (the
7  * "License"); you may not use this file except in compliance
8  * with the License.  You may obtain a copy of the License at
9  *
10  *     http://www.apache.org/licenses/LICENSE-2.0
11  *
12  * Unless required by applicable law or agreed to in writing, software
13  * distributed under the License is distributed on an "AS IS" BASIS,
14  * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15  * See the License for the specific language governing permissions and
16  * limitations under the License.
17  */
18 package org.apache.hadoop.hdfs.server.namenode.snapshot;
19 
20 import java.io.DataOutput;
21 import java.io.IOException;
22 import java.util.ArrayDeque;
23 import java.util.Deque;
24 import java.util.HashMap;
25 import java.util.Iterator;
26 import java.util.List;
27 import java.util.Map;
28 
29 import org.apache.hadoop.classification.InterfaceAudience;
30 import org.apache.hadoop.hdfs.protocol.QuotaExceededException;
31 import org.apache.hadoop.hdfs.server.blockmanagement.BlockStoragePolicySuite;
32 import org.apache.hadoop.hdfs.server.namenode.AclStorage;
33 import org.apache.hadoop.hdfs.server.namenode.Content;
34 import org.apache.hadoop.hdfs.server.namenode.ContentCounts;
35 import org.apache.hadoop.hdfs.server.namenode.ContentSummaryComputationContext;
36 import org.apache.hadoop.hdfs.server.namenode.FSImageSerialization;
37 import org.apache.hadoop.hdfs.server.namenode.INode;
38 import org.apache.hadoop.hdfs.server.namenode.INode.BlocksMapUpdateInfo;
39 import org.apache.hadoop.hdfs.server.namenode.INodeDirectory;
40 import org.apache.hadoop.hdfs.server.namenode.INodeDirectoryAttributes;
41 import org.apache.hadoop.hdfs.server.namenode.INodeFile;
42 import org.apache.hadoop.hdfs.server.namenode.INodeReference;
43 import org.apache.hadoop.hdfs.server.namenode.QuotaCounts;
44 import org.apache.hadoop.hdfs.server.namenode.snapshot.SnapshotFSImageFormat.ReferenceMap;
45 import org.apache.hadoop.hdfs.util.Diff;
46 import org.apache.hadoop.hdfs.util.Diff.Container;
47 import org.apache.hadoop.hdfs.util.Diff.ListType;
48 import org.apache.hadoop.hdfs.util.Diff.UndoInfo;
49 import org.apache.hadoop.hdfs.util.ReadOnlyList;
50 
51 import com.google.common.base.Preconditions;
52 
53 /**
54  * Feature used to store and process the snapshot diff information for a
55  * directory. In particular, it contains a directory diff list recording changes
56  * made to the directory and its children for each snapshot.
57  */
58 @InterfaceAudience.Private
59 public class DirectoryWithSnapshotFeature implements INode.Feature {
60   /**
61    * The difference between the current state and a previous snapshot
62    * of the children list of an INodeDirectory.
63    */
64   static class ChildrenDiff extends Diff<byte[], INode> {
ChildrenDiff()65     ChildrenDiff() {}
66 
ChildrenDiff(final List<INode> created, final List<INode> deleted)67     private ChildrenDiff(final List<INode> created, final List<INode> deleted) {
68       super(created, deleted);
69     }
70 
71     /**
72      * Replace the given child from the created/deleted list.
73      * @return true if the child is replaced; false if the child is not found.
74      */
replace(final ListType type, final INode oldChild, final INode newChild)75     private boolean replace(final ListType type,
76         final INode oldChild, final INode newChild) {
77       final List<INode> list = getList(type);
78       final int i = search(list, oldChild.getLocalNameBytes());
79       if (i < 0 || list.get(i).getId() != oldChild.getId()) {
80         return false;
81       }
82 
83       final INode removed = list.set(i, newChild);
84       Preconditions.checkState(removed == oldChild);
85       return true;
86     }
87 
removeChild(ListType type, final INode child)88     private boolean removeChild(ListType type, final INode child) {
89       final List<INode> list = getList(type);
90       final int i = searchIndex(type, child.getLocalNameBytes());
91       if (i >= 0 && list.get(i) == child) {
92         list.remove(i);
93         return true;
94       }
95       return false;
96     }
97 
98     /** clear the created list */
destroyCreatedList( final BlockStoragePolicySuite bsps, final INodeDirectory currentINode, final BlocksMapUpdateInfo collectedBlocks, final List<INode> removedINodes)99     private QuotaCounts destroyCreatedList(
100         final BlockStoragePolicySuite bsps,
101         final INodeDirectory currentINode,
102         final BlocksMapUpdateInfo collectedBlocks,
103         final List<INode> removedINodes) {
104       QuotaCounts counts = new QuotaCounts.Builder().build();
105       final List<INode> createdList = getList(ListType.CREATED);
106       for (INode c : createdList) {
107         c.computeQuotaUsage(bsps, counts, true);
108         c.destroyAndCollectBlocks(bsps, collectedBlocks, removedINodes);
109         // c should be contained in the children list, remove it
110         currentINode.removeChild(c);
111       }
112       createdList.clear();
113       return counts;
114     }
115 
116     /** clear the deleted list */
destroyDeletedList( final BlockStoragePolicySuite bsps, final BlocksMapUpdateInfo collectedBlocks, final List<INode> removedINodes)117     private QuotaCounts destroyDeletedList(
118         final BlockStoragePolicySuite bsps,
119         final BlocksMapUpdateInfo collectedBlocks,
120         final List<INode> removedINodes) {
121       QuotaCounts counts = new QuotaCounts.Builder().build();
122       final List<INode> deletedList = getList(ListType.DELETED);
123       for (INode d : deletedList) {
124         d.computeQuotaUsage(bsps, counts, false);
125         d.destroyAndCollectBlocks(bsps, collectedBlocks, removedINodes);
126       }
127       deletedList.clear();
128       return counts;
129     }
130 
131     /** Serialize {@link #created} */
writeCreated(DataOutput out)132     private void writeCreated(DataOutput out) throws IOException {
133       final List<INode> created = getList(ListType.CREATED);
134       out.writeInt(created.size());
135       for (INode node : created) {
136         // For INode in created list, we only need to record its local name
137         byte[] name = node.getLocalNameBytes();
138         out.writeShort(name.length);
139         out.write(name);
140       }
141     }
142 
143     /** Serialize {@link #deleted} */
writeDeleted(DataOutput out, ReferenceMap referenceMap)144     private void writeDeleted(DataOutput out,
145         ReferenceMap referenceMap) throws IOException {
146       final List<INode> deleted = getList(ListType.DELETED);
147       out.writeInt(deleted.size());
148       for (INode node : deleted) {
149         FSImageSerialization.saveINode2Image(node, out, true, referenceMap);
150       }
151     }
152 
153     /** Serialize to out */
write(DataOutput out, ReferenceMap referenceMap )154     private void write(DataOutput out, ReferenceMap referenceMap
155         ) throws IOException {
156       writeCreated(out);
157       writeDeleted(out, referenceMap);
158     }
159 
160     /** Get the list of INodeDirectory contained in the deleted list */
getDirsInDeleted(List<INodeDirectory> dirList)161     private void getDirsInDeleted(List<INodeDirectory> dirList) {
162       for (INode node : getList(ListType.DELETED)) {
163         if (node.isDirectory()) {
164           dirList.add(node.asDirectory());
165         }
166       }
167     }
168   }
169 
170   /**
171    * The difference of an {@link INodeDirectory} between two snapshots.
172    */
173   public static class DirectoryDiff extends
174       AbstractINodeDiff<INodeDirectory, INodeDirectoryAttributes, DirectoryDiff> {
175     /** The size of the children list at snapshot creation time. */
176     private final int childrenSize;
177     /** The children list diff. */
178     private final ChildrenDiff diff;
179     private boolean isSnapshotRoot = false;
180 
DirectoryDiff(int snapshotId, INodeDirectory dir)181     private DirectoryDiff(int snapshotId, INodeDirectory dir) {
182       super(snapshotId, null, null);
183 
184       this.childrenSize = dir.getChildrenList(Snapshot.CURRENT_STATE_ID).size();
185       this.diff = new ChildrenDiff();
186     }
187 
188     /** Constructor used by FSImage loading */
DirectoryDiff(int snapshotId, INodeDirectoryAttributes snapshotINode, DirectoryDiff posteriorDiff, int childrenSize, List<INode> createdList, List<INode> deletedList, boolean isSnapshotRoot)189     DirectoryDiff(int snapshotId, INodeDirectoryAttributes snapshotINode,
190         DirectoryDiff posteriorDiff, int childrenSize, List<INode> createdList,
191         List<INode> deletedList, boolean isSnapshotRoot) {
192       super(snapshotId, snapshotINode, posteriorDiff);
193       this.childrenSize = childrenSize;
194       this.diff = new ChildrenDiff(createdList, deletedList);
195       this.isSnapshotRoot = isSnapshotRoot;
196     }
197 
getChildrenDiff()198     public ChildrenDiff getChildrenDiff() {
199       return diff;
200     }
201 
setSnapshotRoot(INodeDirectoryAttributes root)202     void setSnapshotRoot(INodeDirectoryAttributes root) {
203       this.snapshotINode = root;
204       this.isSnapshotRoot = true;
205     }
206 
isSnapshotRoot()207     boolean isSnapshotRoot() {
208       return isSnapshotRoot;
209     }
210 
211     @Override
combinePosteriorAndCollectBlocks( final BlockStoragePolicySuite bsps, final INodeDirectory currentDir, final DirectoryDiff posterior, final BlocksMapUpdateInfo collectedBlocks, final List<INode> removedINodes)212     QuotaCounts combinePosteriorAndCollectBlocks(
213         final BlockStoragePolicySuite bsps,
214         final INodeDirectory currentDir, final DirectoryDiff posterior,
215         final BlocksMapUpdateInfo collectedBlocks,
216         final List<INode> removedINodes) {
217       final QuotaCounts counts = new QuotaCounts.Builder().build();
218       diff.combinePosterior(posterior.diff, new Diff.Processor<INode>() {
219         /** Collect blocks for deleted files. */
220         @Override
221         public void process(INode inode) {
222           if (inode != null) {
223             inode.computeQuotaUsage(bsps, counts, false);
224             inode.destroyAndCollectBlocks(bsps, collectedBlocks, removedINodes);
225           }
226         }
227       });
228       return counts;
229     }
230 
231     /**
232      * @return The children list of a directory in a snapshot.
233      *         Since the snapshot is read-only, the logical view of the list is
234      *         never changed although the internal data structure may mutate.
235      */
getChildrenList(final INodeDirectory currentDir)236     private ReadOnlyList<INode> getChildrenList(final INodeDirectory currentDir) {
237       return new ReadOnlyList<INode>() {
238         private List<INode> children = null;
239 
240         private List<INode> initChildren() {
241           if (children == null) {
242             final ChildrenDiff combined = new ChildrenDiff();
243             for (DirectoryDiff d = DirectoryDiff.this; d != null;
244                 d = d.getPosterior()) {
245               combined.combinePosterior(d.diff, null);
246             }
247             children = combined.apply2Current(ReadOnlyList.Util.asList(
248                 currentDir.getChildrenList(Snapshot.CURRENT_STATE_ID)));
249           }
250           return children;
251         }
252 
253         @Override
254         public Iterator<INode> iterator() {
255           return initChildren().iterator();
256         }
257 
258         @Override
259         public boolean isEmpty() {
260           return childrenSize == 0;
261         }
262 
263         @Override
264         public int size() {
265           return childrenSize;
266         }
267 
268         @Override
269         public INode get(int i) {
270           return initChildren().get(i);
271         }
272       };
273     }
274 
275     /** @return the child with the given name. */
getChild(byte[] name, boolean checkPosterior, INodeDirectory currentDir)276     INode getChild(byte[] name, boolean checkPosterior,
277         INodeDirectory currentDir) {
278       for(DirectoryDiff d = this; ; d = d.getPosterior()) {
279         final Container<INode> returned = d.diff.accessPrevious(name);
280         if (returned != null) {
281           // the diff is able to determine the inode
282           return returned.getElement();
283         } else if (!checkPosterior) {
284           // Since checkPosterior is false, return null, i.e. not found.
285           return null;
286         } else if (d.getPosterior() == null) {
287           // no more posterior diff, get from current inode.
288           return currentDir.getChild(name, Snapshot.CURRENT_STATE_ID);
289         }
290       }
291     }
292 
293     @Override
toString()294     public String toString() {
295       return super.toString() + " childrenSize=" + childrenSize + ", " + diff;
296     }
297 
getChildrenSize()298     int getChildrenSize() {
299       return childrenSize;
300     }
301 
302     @Override
write(DataOutput out, ReferenceMap referenceMap)303     void write(DataOutput out, ReferenceMap referenceMap) throws IOException {
304       writeSnapshot(out);
305       out.writeInt(childrenSize);
306 
307       // Write snapshotINode
308       out.writeBoolean(isSnapshotRoot);
309       if (!isSnapshotRoot) {
310         if (snapshotINode != null) {
311           out.writeBoolean(true);
312           FSImageSerialization.writeINodeDirectoryAttributes(snapshotINode, out);
313         } else {
314           out.writeBoolean(false);
315         }
316       }
317       // Write diff. Node need to write poseriorDiff, since diffs is a list.
318       diff.write(out, referenceMap);
319     }
320 
321     @Override
destroyDiffAndCollectBlocks( BlockStoragePolicySuite bsps, INodeDirectory currentINode, BlocksMapUpdateInfo collectedBlocks, final List<INode> removedINodes)322     QuotaCounts destroyDiffAndCollectBlocks(
323         BlockStoragePolicySuite bsps, INodeDirectory currentINode,
324         BlocksMapUpdateInfo collectedBlocks, final List<INode> removedINodes) {
325       // this diff has been deleted
326       QuotaCounts counts = new QuotaCounts.Builder().build();
327       counts.add(diff.destroyDeletedList(bsps, collectedBlocks, removedINodes));
328       INodeDirectoryAttributes snapshotINode = getSnapshotINode();
329       if (snapshotINode != null && snapshotINode.getAclFeature() != null) {
330         AclStorage.removeAclFeature(snapshotINode.getAclFeature());
331       }
332       return counts;
333     }
334   }
335 
336   /** A list of directory diffs. */
337   public static class DirectoryDiffList
338       extends AbstractINodeDiffList<INodeDirectory, INodeDirectoryAttributes, DirectoryDiff> {
339 
340     @Override
createDiff(int snapshot, INodeDirectory currentDir)341     DirectoryDiff createDiff(int snapshot, INodeDirectory currentDir) {
342       return new DirectoryDiff(snapshot, currentDir);
343     }
344 
345     @Override
createSnapshotCopy(INodeDirectory currentDir)346     INodeDirectoryAttributes createSnapshotCopy(INodeDirectory currentDir) {
347       return currentDir.isQuotaSet()?
348           new INodeDirectoryAttributes.CopyWithQuota(currentDir)
349         : new INodeDirectoryAttributes.SnapshotCopy(currentDir);
350     }
351 
352     /** Replace the given child in the created/deleted list, if there is any. */
replaceChild(final ListType type, final INode oldChild, final INode newChild)353     public boolean replaceChild(final ListType type, final INode oldChild,
354         final INode newChild) {
355       final List<DirectoryDiff> diffList = asList();
356       for(int i = diffList.size() - 1; i >= 0; i--) {
357         final ChildrenDiff diff = diffList.get(i).diff;
358         if (diff.replace(type, oldChild, newChild)) {
359           return true;
360         }
361       }
362       return false;
363     }
364 
365     /** Remove the given child in the created/deleted list, if there is any. */
removeChild(final ListType type, final INode child)366     public boolean removeChild(final ListType type, final INode child) {
367       final List<DirectoryDiff> diffList = asList();
368       for(int i = diffList.size() - 1; i >= 0; i--) {
369         final ChildrenDiff diff = diffList.get(i).diff;
370         if (diff.removeChild(type, child)) {
371           return true;
372         }
373       }
374       return false;
375     }
376 
377     /**
378      * Find the corresponding snapshot whose deleted list contains the given
379      * inode.
380      * @return the id of the snapshot. {@link Snapshot#NO_SNAPSHOT_ID} if the
381      * given inode is not in any of the snapshot.
382      */
findSnapshotDeleted(final INode child)383     public int findSnapshotDeleted(final INode child) {
384       final List<DirectoryDiff> diffList = asList();
385       for(int i = diffList.size() - 1; i >= 0; i--) {
386         final ChildrenDiff diff = diffList.get(i).diff;
387         final int d = diff.searchIndex(ListType.DELETED,
388             child.getLocalNameBytes());
389         if (d >= 0 && diff.getList(ListType.DELETED).get(d) == child) {
390           return diffList.get(i).getSnapshotId();
391         }
392       }
393       return Snapshot.NO_SNAPSHOT_ID;
394     }
395   }
396 
cloneDiffList(List<INode> diffList)397   private static Map<INode, INode> cloneDiffList(List<INode> diffList) {
398     if (diffList == null || diffList.size() == 0) {
399       return null;
400     }
401     Map<INode, INode> map = new HashMap<INode, INode>(diffList.size());
402     for (INode node : diffList) {
403       map.put(node, node);
404     }
405     return map;
406   }
407 
408   /**
409    * Destroy a subtree under a DstReference node.
410    */
destroyDstSubtree( final BlockStoragePolicySuite bsps, INode inode, final int snapshot, final int prior, final BlocksMapUpdateInfo collectedBlocks, final List<INode> removedINodes)411   public static void destroyDstSubtree(
412       final BlockStoragePolicySuite bsps, INode inode, final int snapshot,
413       final int prior, final BlocksMapUpdateInfo collectedBlocks,
414       final List<INode> removedINodes) throws QuotaExceededException {
415     Preconditions.checkArgument(prior != Snapshot.NO_SNAPSHOT_ID);
416     if (inode.isReference()) {
417       if (inode instanceof INodeReference.WithName
418           && snapshot != Snapshot.CURRENT_STATE_ID) {
419         // this inode has been renamed before the deletion of the DstReference
420         // subtree
421         inode.cleanSubtree(bsps, snapshot, prior, collectedBlocks, removedINodes);
422       } else {
423         // for DstReference node, continue this process to its subtree
424         destroyDstSubtree(bsps, inode.asReference().getReferredINode(), snapshot,
425             prior, collectedBlocks, removedINodes);
426       }
427     } else if (inode.isFile()) {
428       inode.cleanSubtree(bsps, snapshot, prior, collectedBlocks, removedINodes);
429     } else if (inode.isDirectory()) {
430       Map<INode, INode> excludedNodes = null;
431       INodeDirectory dir = inode.asDirectory();
432       DirectoryWithSnapshotFeature sf = dir.getDirectoryWithSnapshotFeature();
433       if (sf != null) {
434         DirectoryDiffList diffList = sf.getDiffs();
435         DirectoryDiff priorDiff = diffList.getDiffById(prior);
436         if (priorDiff != null && priorDiff.getSnapshotId() == prior) {
437           List<INode> dList = priorDiff.diff.getList(ListType.DELETED);
438           excludedNodes = cloneDiffList(dList);
439         }
440 
441         if (snapshot != Snapshot.CURRENT_STATE_ID) {
442           diffList.deleteSnapshotDiff(bsps, snapshot, prior, dir, collectedBlocks,
443               removedINodes);
444         }
445         priorDiff = diffList.getDiffById(prior);
446         if (priorDiff != null && priorDiff.getSnapshotId() == prior) {
447           priorDiff.diff.destroyCreatedList(bsps, dir, collectedBlocks,
448               removedINodes);
449         }
450       }
451       for (INode child : inode.asDirectory().getChildrenList(prior)) {
452         if (excludedNodes != null && excludedNodes.containsKey(child)) {
453           continue;
454         }
455         destroyDstSubtree(bsps, child, snapshot, prior, collectedBlocks,
456             removedINodes);
457       }
458     }
459   }
460 
461   /**
462    * Clean an inode while we move it from the deleted list of post to the
463    * deleted list of prior.
464    * @param bsps The block storage policy suite.
465    * @param inode The inode to clean.
466    * @param post The post snapshot.
467    * @param prior The id of the prior snapshot.
468    * @param collectedBlocks Used to collect blocks for later deletion.
469    * @return Quota usage update.
470    */
cleanDeletedINode( final BlockStoragePolicySuite bsps, INode inode, final int post, final int prior, final BlocksMapUpdateInfo collectedBlocks, final List<INode> removedINodes)471   private static QuotaCounts cleanDeletedINode(
472       final BlockStoragePolicySuite bsps, INode inode,
473       final int post, final int prior,
474       final BlocksMapUpdateInfo collectedBlocks,
475       final List<INode> removedINodes) {
476     QuotaCounts counts = new QuotaCounts.Builder().build();
477     Deque<INode> queue = new ArrayDeque<INode>();
478     queue.addLast(inode);
479     while (!queue.isEmpty()) {
480       INode topNode = queue.pollFirst();
481       if (topNode instanceof INodeReference.WithName) {
482         INodeReference.WithName wn = (INodeReference.WithName) topNode;
483         if (wn.getLastSnapshotId() >= post) {
484           wn.cleanSubtree(bsps, post, prior, collectedBlocks, removedINodes);
485         }
486         // For DstReference node, since the node is not in the created list of
487         // prior, we should treat it as regular file/dir
488       } else if (topNode.isFile() && topNode.asFile().isWithSnapshot()) {
489         INodeFile file = topNode.asFile();
490         counts.add(file.getDiffs().deleteSnapshotDiff(bsps, post, prior, file,
491             collectedBlocks, removedINodes));
492       } else if (topNode.isDirectory()) {
493         INodeDirectory dir = topNode.asDirectory();
494         ChildrenDiff priorChildrenDiff = null;
495         DirectoryWithSnapshotFeature sf = dir.getDirectoryWithSnapshotFeature();
496         if (sf != null) {
497           // delete files/dirs created after prior. Note that these
498           // files/dirs, along with inode, were deleted right after post.
499           DirectoryDiff priorDiff = sf.getDiffs().getDiffById(prior);
500           if (priorDiff != null && priorDiff.getSnapshotId() == prior) {
501             priorChildrenDiff = priorDiff.getChildrenDiff();
502             counts.add(priorChildrenDiff.destroyCreatedList(bsps, dir,
503                 collectedBlocks, removedINodes));
504           }
505         }
506 
507         for (INode child : dir.getChildrenList(prior)) {
508           if (priorChildrenDiff != null
509               && priorChildrenDiff.search(ListType.DELETED,
510                   child.getLocalNameBytes()) != null) {
511             continue;
512           }
513           queue.addLast(child);
514         }
515       }
516     }
517     return counts;
518   }
519 
520   /** Diff list sorted by snapshot IDs, i.e. in chronological order. */
521   private final DirectoryDiffList diffs;
522 
DirectoryWithSnapshotFeature(DirectoryDiffList diffs)523   public DirectoryWithSnapshotFeature(DirectoryDiffList diffs) {
524     this.diffs = diffs != null ? diffs : new DirectoryDiffList();
525   }
526 
527   /** @return the last snapshot. */
getLastSnapshotId()528   public int getLastSnapshotId() {
529     return diffs.getLastSnapshotId();
530   }
531 
532   /** @return the snapshot diff list. */
getDiffs()533   public DirectoryDiffList getDiffs() {
534     return diffs;
535   }
536 
537   /**
538    * Get all the directories that are stored in some snapshot but not in the
539    * current children list. These directories are equivalent to the directories
540    * stored in the deletes lists.
541    */
getSnapshotDirectory(List<INodeDirectory> snapshotDir)542   public void getSnapshotDirectory(List<INodeDirectory> snapshotDir) {
543     for (DirectoryDiff sdiff : diffs) {
544       sdiff.getChildrenDiff().getDirsInDeleted(snapshotDir);
545     }
546   }
547 
548   /**
549    * Add an inode into parent's children list. The caller of this method needs
550    * to make sure that parent is in the given snapshot "latest".
551    */
addChild(INodeDirectory parent, INode inode, boolean setModTime, int latestSnapshotId)552   public boolean addChild(INodeDirectory parent, INode inode,
553       boolean setModTime, int latestSnapshotId) throws QuotaExceededException {
554     ChildrenDiff diff = diffs.checkAndAddLatestSnapshotDiff(latestSnapshotId,
555         parent).diff;
556     int undoInfo = diff.create(inode);
557 
558     final boolean added = parent.addChild(inode, setModTime,
559         Snapshot.CURRENT_STATE_ID);
560     if (!added) {
561       diff.undoCreate(inode, undoInfo);
562     }
563     return added;
564   }
565 
566   /**
567    * Remove an inode from parent's children list. The caller of this method
568    * needs to make sure that parent is in the given snapshot "latest".
569    */
removeChild(INodeDirectory parent, INode child, int latestSnapshotId)570   public boolean removeChild(INodeDirectory parent, INode child,
571       int latestSnapshotId) {
572     // For a directory that is not a renamed node, if isInLatestSnapshot returns
573     // false, the directory is not in the latest snapshot, thus we do not need
574     // to record the removed child in any snapshot.
575     // For a directory that was moved/renamed, note that if the directory is in
576     // any of the previous snapshots, we will create a reference node for the
577     // directory while rename, and isInLatestSnapshot will return true in that
578     // scenario (if all previous snapshots have been deleted, isInLatestSnapshot
579     // still returns false). Thus if isInLatestSnapshot returns false, the
580     // directory node cannot be in any snapshot (not in current tree, nor in
581     // previous src tree). Thus we do not need to record the removed child in
582     // any snapshot.
583     ChildrenDiff diff = diffs.checkAndAddLatestSnapshotDiff(latestSnapshotId,
584         parent).diff;
585     UndoInfo<INode> undoInfo = diff.delete(child);
586 
587     final boolean removed = parent.removeChild(child);
588     if (!removed && undoInfo != null) {
589       // remove failed, undo
590       diff.undoDelete(child, undoInfo);
591     }
592     return removed;
593   }
594 
595   /**
596    * @return If there is no corresponding directory diff for the given
597    *         snapshot, this means that the current children list should be
598    *         returned for the snapshot. Otherwise we calculate the children list
599    *         for the snapshot and return it.
600    */
getChildrenList(INodeDirectory currentINode, final int snapshotId)601   public ReadOnlyList<INode> getChildrenList(INodeDirectory currentINode,
602       final int snapshotId) {
603     final DirectoryDiff diff = diffs.getDiffById(snapshotId);
604     return diff != null ? diff.getChildrenList(currentINode) : currentINode
605         .getChildrenList(Snapshot.CURRENT_STATE_ID);
606   }
607 
getChild(INodeDirectory currentINode, byte[] name, int snapshotId)608   public INode getChild(INodeDirectory currentINode, byte[] name,
609       int snapshotId) {
610     final DirectoryDiff diff = diffs.getDiffById(snapshotId);
611     return diff != null ? diff.getChild(name, true, currentINode)
612         : currentINode.getChild(name, Snapshot.CURRENT_STATE_ID);
613   }
614 
615   /** Used to record the modification of a symlink node */
saveChild2Snapshot(INodeDirectory currentINode, final INode child, final int latestSnapshotId, final INode snapshotCopy)616   public INode saveChild2Snapshot(INodeDirectory currentINode,
617       final INode child, final int latestSnapshotId, final INode snapshotCopy) {
618     Preconditions.checkArgument(!child.isDirectory(),
619         "child is a directory, child=%s", child);
620     Preconditions.checkArgument(latestSnapshotId != Snapshot.CURRENT_STATE_ID);
621 
622     final DirectoryDiff diff = diffs.checkAndAddLatestSnapshotDiff(
623         latestSnapshotId, currentINode);
624     if (diff.getChild(child.getLocalNameBytes(), false, currentINode) != null) {
625       // it was already saved in the latest snapshot earlier.
626       return child;
627     }
628 
629     diff.diff.modify(snapshotCopy, child);
630     return child;
631   }
632 
clear(BlockStoragePolicySuite bsps, INodeDirectory currentINode, final BlocksMapUpdateInfo collectedBlocks, final List<INode> removedINodes)633   public void clear(BlockStoragePolicySuite bsps, INodeDirectory currentINode,
634       final BlocksMapUpdateInfo collectedBlocks, final List<INode> removedINodes) {
635     // destroy its diff list
636     for (DirectoryDiff diff : diffs) {
637       diff.destroyDiffAndCollectBlocks(bsps, currentINode, collectedBlocks,
638         removedINodes);
639     }
640     diffs.clear();
641   }
642 
computeQuotaUsage4CurrentDirectory( BlockStoragePolicySuite bsps, byte storagePolicyId, QuotaCounts counts)643   public QuotaCounts computeQuotaUsage4CurrentDirectory(
644       BlockStoragePolicySuite bsps, byte storagePolicyId,
645       QuotaCounts counts) {
646     for(DirectoryDiff d : diffs) {
647       for(INode deleted : d.getChildrenDiff().getList(ListType.DELETED)) {
648         final byte childPolicyId = deleted.getStoragePolicyIDForQuota(storagePolicyId);
649         deleted.computeQuotaUsage(bsps, childPolicyId, counts, false,
650             Snapshot.CURRENT_STATE_ID);
651       }
652     }
653     return counts;
654   }
655 
computeContentSummary4Snapshot(final BlockStoragePolicySuite bsps, final ContentCounts counts)656   public void computeContentSummary4Snapshot(final BlockStoragePolicySuite bsps,
657       final ContentCounts counts) {
658     // Create a new blank summary context for blocking processing of subtree.
659     ContentSummaryComputationContext summary =
660         new ContentSummaryComputationContext(bsps);
661     for(DirectoryDiff d : diffs) {
662       for(INode deleted : d.getChildrenDiff().getList(ListType.DELETED)) {
663         deleted.computeContentSummary(summary);
664       }
665     }
666     // Add the counts from deleted trees.
667     counts.addContents(summary.getCounts());
668     // Add the deleted directory count.
669     counts.addContent(Content.DIRECTORY, diffs.asList().size());
670   }
671 
672   /**
673    * Compute the difference between Snapshots.
674    *
675    * @param fromSnapshot Start point of the diff computation. Null indicates
676    *          current tree.
677    * @param toSnapshot End point of the diff computation. Null indicates current
678    *          tree.
679    * @param diff Used to capture the changes happening to the children. Note
680    *          that the diff still represents (later_snapshot - earlier_snapshot)
681    *          although toSnapshot can be before fromSnapshot.
682    * @param currentINode The {@link INodeDirectory} this feature belongs to.
683    * @return Whether changes happened between the startSnapshot and endSnaphsot.
684    */
computeDiffBetweenSnapshots(Snapshot fromSnapshot, Snapshot toSnapshot, ChildrenDiff diff, INodeDirectory currentINode)685   boolean computeDiffBetweenSnapshots(Snapshot fromSnapshot,
686       Snapshot toSnapshot, ChildrenDiff diff, INodeDirectory currentINode) {
687     int[] diffIndexPair = diffs.changedBetweenSnapshots(fromSnapshot,
688         toSnapshot);
689     if (diffIndexPair == null) {
690       return false;
691     }
692     int earlierDiffIndex = diffIndexPair[0];
693     int laterDiffIndex = diffIndexPair[1];
694 
695     boolean dirMetadataChanged = false;
696     INodeDirectoryAttributes dirCopy = null;
697     List<DirectoryDiff> difflist = diffs.asList();
698     for (int i = earlierDiffIndex; i < laterDiffIndex; i++) {
699       DirectoryDiff sdiff = difflist.get(i);
700       diff.combinePosterior(sdiff.diff, null);
701       if (!dirMetadataChanged && sdiff.snapshotINode != null) {
702         if (dirCopy == null) {
703           dirCopy = sdiff.snapshotINode;
704         } else if (!dirCopy.metadataEquals(sdiff.snapshotINode)) {
705           dirMetadataChanged = true;
706         }
707       }
708     }
709 
710     if (!diff.isEmpty() || dirMetadataChanged) {
711       return true;
712     } else if (dirCopy != null) {
713       for (int i = laterDiffIndex; i < difflist.size(); i++) {
714         if (!dirCopy.metadataEquals(difflist.get(i).snapshotINode)) {
715           return true;
716         }
717       }
718       return !dirCopy.metadataEquals(currentINode);
719     } else {
720       return false;
721     }
722   }
723 
cleanDirectory(final BlockStoragePolicySuite bsps, final INodeDirectory currentINode, final int snapshot, int prior, final BlocksMapUpdateInfo collectedBlocks, final List<INode> removedINodes)724   public QuotaCounts cleanDirectory(final BlockStoragePolicySuite bsps, final INodeDirectory currentINode,
725       final int snapshot, int prior,
726       final BlocksMapUpdateInfo collectedBlocks,
727       final List<INode> removedINodes) {
728     QuotaCounts counts = new QuotaCounts.Builder().build();
729     Map<INode, INode> priorCreated = null;
730     Map<INode, INode> priorDeleted = null;
731     if (snapshot == Snapshot.CURRENT_STATE_ID) { // delete the current directory
732       currentINode.recordModification(prior);
733       // delete everything in created list
734       DirectoryDiff lastDiff = diffs.getLast();
735       if (lastDiff != null) {
736         counts.add(lastDiff.diff.destroyCreatedList(bsps, currentINode,
737             collectedBlocks, removedINodes));
738       }
739       counts.add(currentINode.cleanSubtreeRecursively(bsps, snapshot, prior,
740           collectedBlocks, removedINodes, priorDeleted));
741     } else {
742       // update prior
743       prior = getDiffs().updatePrior(snapshot, prior);
744       // if there is a snapshot diff associated with prior, we need to record
745       // its original created and deleted list before deleting post
746       if (prior != Snapshot.NO_SNAPSHOT_ID) {
747         DirectoryDiff priorDiff = this.getDiffs().getDiffById(prior);
748         if (priorDiff != null && priorDiff.getSnapshotId() == prior) {
749           List<INode> cList = priorDiff.diff.getList(ListType.CREATED);
750           List<INode> dList = priorDiff.diff.getList(ListType.DELETED);
751           priorCreated = cloneDiffList(cList);
752           priorDeleted = cloneDiffList(dList);
753         }
754       }
755 
756       counts.add(getDiffs().deleteSnapshotDiff(bsps, snapshot, prior,
757           currentINode, collectedBlocks, removedINodes));
758       counts.add(currentINode.cleanSubtreeRecursively(bsps, snapshot, prior,
759           collectedBlocks, removedINodes, priorDeleted));
760 
761       // check priorDiff again since it may be created during the diff deletion
762       if (prior != Snapshot.NO_SNAPSHOT_ID) {
763         DirectoryDiff priorDiff = this.getDiffs().getDiffById(prior);
764         if (priorDiff != null && priorDiff.getSnapshotId() == prior) {
765           // For files/directories created between "prior" and "snapshot",
766           // we need to clear snapshot copies for "snapshot". Note that we must
767           // use null as prior in the cleanSubtree call. Files/directories that
768           // were created before "prior" will be covered by the later
769           // cleanSubtreeRecursively call.
770           if (priorCreated != null) {
771             // we only check the node originally in prior's created list
772             for (INode cNode : priorDiff.getChildrenDiff().getList(
773                 ListType.CREATED)) {
774               if (priorCreated.containsKey(cNode)) {
775                 counts.add(cNode.cleanSubtree(bsps, snapshot, Snapshot.NO_SNAPSHOT_ID,
776                     collectedBlocks, removedINodes));
777               }
778             }
779           }
780 
781           // When a directory is moved from the deleted list of the posterior
782           // diff to the deleted list of this diff, we need to destroy its
783           // descendants that were 1) created after taking this diff and 2)
784           // deleted after taking posterior diff.
785 
786           // For files moved from posterior's deleted list, we also need to
787           // delete its snapshot copy associated with the posterior snapshot.
788 
789           for (INode dNode : priorDiff.getChildrenDiff().getList(
790               ListType.DELETED)) {
791             if (priorDeleted == null || !priorDeleted.containsKey(dNode)) {
792               counts.add(cleanDeletedINode(bsps, dNode, snapshot, prior,
793                   collectedBlocks, removedINodes));
794             }
795           }
796         }
797       }
798     }
799 
800     if (currentINode.isQuotaSet()) {
801       currentINode.getDirectoryWithQuotaFeature().addSpaceConsumed2Cache(
802           counts.negation());
803     }
804     return counts;
805   }
806 }
807