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.DataInput;
21 import java.io.DataOutput;
22 import java.io.IOException;
23 import java.util.ArrayList;
24 import java.util.HashMap;
25 import java.util.List;
26 import java.util.Map;
27 
28 import org.apache.hadoop.hdfs.DFSUtil;
29 import org.apache.hadoop.hdfs.server.namenode.FSImageFormat;
30 import org.apache.hadoop.hdfs.server.namenode.FSImageSerialization;
31 import org.apache.hadoop.hdfs.server.namenode.INode;
32 import org.apache.hadoop.hdfs.server.namenode.INodeAttributes;
33 import org.apache.hadoop.hdfs.server.namenode.INodeDirectory;
34 import org.apache.hadoop.hdfs.server.namenode.INodeDirectoryAttributes;
35 import org.apache.hadoop.hdfs.server.namenode.INodeFile;
36 import org.apache.hadoop.hdfs.server.namenode.INodeFileAttributes;
37 import org.apache.hadoop.hdfs.server.namenode.INodeReference;
38 import org.apache.hadoop.hdfs.server.namenode.snapshot.DirectoryWithSnapshotFeature.DirectoryDiff;
39 import org.apache.hadoop.hdfs.server.namenode.snapshot.DirectoryWithSnapshotFeature.DirectoryDiffList;
40 import org.apache.hadoop.hdfs.tools.snapshot.SnapshotDiff;
41 import org.apache.hadoop.hdfs.util.Diff.ListType;
42 import org.apache.hadoop.hdfs.util.ReadOnlyList;
43 
44 import com.google.common.base.Preconditions;
45 
46 /**
47  * A helper class defining static methods for reading/writing snapshot related
48  * information from/to FSImage.
49  */
50 public class SnapshotFSImageFormat {
51   /**
52    * Save snapshots and snapshot quota for a snapshottable directory.
53    * @param current The directory that the snapshots belongs to.
54    * @param out The {@link DataOutput} to write.
55    * @throws IOException
56    */
saveSnapshots(INodeDirectory current, DataOutput out)57   public static void saveSnapshots(INodeDirectory current, DataOutput out)
58       throws IOException {
59     DirectorySnapshottableFeature sf = current.getDirectorySnapshottableFeature();
60     Preconditions.checkArgument(sf != null);
61     // list of snapshots in snapshotsByNames
62     ReadOnlyList<Snapshot> snapshots = sf.getSnapshotList();
63     out.writeInt(snapshots.size());
64     for (Snapshot s : snapshots) {
65       // write the snapshot id
66       out.writeInt(s.getId());
67     }
68     // snapshot quota
69     out.writeInt(sf.getSnapshotQuota());
70   }
71 
72   /**
73    * Save SnapshotDiff list for an INodeDirectoryWithSnapshot.
74    * @param sNode The directory that the SnapshotDiff list belongs to.
75    * @param out The {@link DataOutput} to write.
76    */
77   private static <N extends INode, A extends INodeAttributes, D extends AbstractINodeDiff<N, A, D>>
saveINodeDiffs(final AbstractINodeDiffList<N, A, D> diffs, final DataOutput out, ReferenceMap referenceMap)78       void saveINodeDiffs(final AbstractINodeDiffList<N, A, D> diffs,
79       final DataOutput out, ReferenceMap referenceMap) throws IOException {
80     // Record the diffs in reversed order, so that we can find the correct
81     // reference for INodes in the created list when loading the FSImage
82     if (diffs == null) {
83       out.writeInt(-1); // no diffs
84     } else {
85       final List<D> list = diffs.asList();
86       final int size = list.size();
87       out.writeInt(size);
88       for (int i = size - 1; i >= 0; i--) {
89         list.get(i).write(out, referenceMap);
90       }
91     }
92   }
93 
saveDirectoryDiffList(final INodeDirectory dir, final DataOutput out, final ReferenceMap referenceMap )94   public static void saveDirectoryDiffList(final INodeDirectory dir,
95       final DataOutput out, final ReferenceMap referenceMap
96       ) throws IOException {
97     saveINodeDiffs(dir.getDiffs(), out, referenceMap);
98   }
99 
saveFileDiffList(final INodeFile file, final DataOutput out)100   public static void saveFileDiffList(final INodeFile file,
101       final DataOutput out) throws IOException {
102     saveINodeDiffs(file.getDiffs(), out, null);
103   }
104 
loadFileDiffList(DataInput in, FSImageFormat.Loader loader)105   public static FileDiffList loadFileDiffList(DataInput in,
106       FSImageFormat.Loader loader) throws IOException {
107     final int size = in.readInt();
108     if (size == -1) {
109       return null;
110     } else {
111       final FileDiffList diffs = new FileDiffList();
112       FileDiff posterior = null;
113       for(int i = 0; i < size; i++) {
114         final FileDiff d = loadFileDiff(posterior, in, loader);
115         diffs.addFirst(d);
116         posterior = d;
117       }
118       return diffs;
119     }
120   }
121 
loadFileDiff(FileDiff posterior, DataInput in, FSImageFormat.Loader loader)122   private static FileDiff loadFileDiff(FileDiff posterior, DataInput in,
123       FSImageFormat.Loader loader) throws IOException {
124     // 1. Read the id of the Snapshot root to identify the Snapshot
125     final Snapshot snapshot = loader.getSnapshot(in);
126 
127     // 2. Load file size
128     final long fileSize = in.readLong();
129 
130     // 3. Load snapshotINode
131     final INodeFileAttributes snapshotINode = in.readBoolean()?
132         loader.loadINodeFileAttributes(in): null;
133 
134     return new FileDiff(snapshot.getId(), snapshotINode, posterior, fileSize);
135   }
136 
137   /**
138    * Load a node stored in the created list from fsimage.
139    * @param createdNodeName The name of the created node.
140    * @param parent The directory that the created list belongs to.
141    * @return The created node.
142    */
loadCreated(byte[] createdNodeName, INodeDirectory parent)143   public static INode loadCreated(byte[] createdNodeName,
144       INodeDirectory parent) throws IOException {
145     // the INode in the created list should be a reference to another INode
146     // in posterior SnapshotDiffs or one of the current children
147     for (DirectoryDiff postDiff : parent.getDiffs()) {
148       final INode d = postDiff.getChildrenDiff().search(ListType.DELETED,
149           createdNodeName);
150       if (d != null) {
151         return d;
152       } // else go to the next SnapshotDiff
153     }
154     // use the current child
155     INode currentChild = parent.getChild(createdNodeName,
156         Snapshot.CURRENT_STATE_ID);
157     if (currentChild == null) {
158       throw new IOException("Cannot find an INode associated with the INode "
159           + DFSUtil.bytes2String(createdNodeName)
160           + " in created list while loading FSImage.");
161     }
162     return currentChild;
163   }
164 
165   /**
166    * Load the created list from fsimage.
167    * @param parent The directory that the created list belongs to.
168    * @param in The {@link DataInput} to read.
169    * @return The created list.
170    */
loadCreatedList(INodeDirectory parent, DataInput in)171   private static List<INode> loadCreatedList(INodeDirectory parent,
172       DataInput in) throws IOException {
173     // read the size of the created list
174     int createdSize = in.readInt();
175     List<INode> createdList = new ArrayList<INode>(createdSize);
176     for (int i = 0; i < createdSize; i++) {
177       byte[] createdNodeName = FSImageSerialization.readLocalName(in);
178       INode created = loadCreated(createdNodeName, parent);
179       createdList.add(created);
180     }
181     return createdList;
182   }
183 
184   /**
185    * Load the deleted list from the fsimage.
186    *
187    * @param parent The directory that the deleted list belongs to.
188    * @param createdList The created list associated with the deleted list in
189    *                    the same Diff.
190    * @param in The {@link DataInput} to read.
191    * @param loader The {@link Loader} instance.
192    * @return The deleted list.
193    */
loadDeletedList(INodeDirectory parent, List<INode> createdList, DataInput in, FSImageFormat.Loader loader)194   private static List<INode> loadDeletedList(INodeDirectory parent,
195       List<INode> createdList, DataInput in, FSImageFormat.Loader loader)
196       throws IOException {
197     int deletedSize = in.readInt();
198     List<INode> deletedList = new ArrayList<INode>(deletedSize);
199     for (int i = 0; i < deletedSize; i++) {
200       final INode deleted = loader.loadINodeWithLocalName(true, in, true);
201       deletedList.add(deleted);
202       // set parent: the parent field of an INode in the deleted list is not
203       // useful, but set the parent here to be consistent with the original
204       // fsdir tree.
205       deleted.setParent(parent);
206       if (deleted.isFile()) {
207         loader.updateBlocksMap(deleted.asFile());
208       }
209     }
210     return deletedList;
211   }
212 
213   /**
214    * Load snapshots and snapshotQuota for a Snapshottable directory.
215    *
216    * @param snapshottableParent
217    *          The snapshottable directory for loading.
218    * @param numSnapshots
219    *          The number of snapshots that the directory has.
220    * @param loader
221    *          The loader
222    */
loadSnapshotList(INodeDirectory snapshottableParent, int numSnapshots, DataInput in, FSImageFormat.Loader loader)223   public static void loadSnapshotList(INodeDirectory snapshottableParent,
224       int numSnapshots, DataInput in, FSImageFormat.Loader loader)
225       throws IOException {
226     DirectorySnapshottableFeature sf = snapshottableParent
227         .getDirectorySnapshottableFeature();
228     Preconditions.checkArgument(sf != null);
229     for (int i = 0; i < numSnapshots; i++) {
230       // read snapshots
231       final Snapshot s = loader.getSnapshot(in);
232       s.getRoot().setParent(snapshottableParent);
233       sf.addSnapshot(s);
234     }
235     int snapshotQuota = in.readInt();
236     snapshottableParent.setSnapshotQuota(snapshotQuota);
237   }
238 
239   /**
240    * Load the {@link SnapshotDiff} list for the INodeDirectoryWithSnapshot
241    * directory.
242    *
243    * @param dir
244    *          The snapshottable directory for loading.
245    * @param in
246    *          The {@link DataInput} instance to read.
247    * @param loader
248    *          The loader
249    */
loadDirectoryDiffList(INodeDirectory dir, DataInput in, FSImageFormat.Loader loader)250   public static void loadDirectoryDiffList(INodeDirectory dir,
251       DataInput in, FSImageFormat.Loader loader) throws IOException {
252     final int size = in.readInt();
253     if (dir.isWithSnapshot()) {
254       DirectoryDiffList diffs = dir.getDiffs();
255       for (int i = 0; i < size; i++) {
256         diffs.addFirst(loadDirectoryDiff(dir, in, loader));
257       }
258     }
259   }
260 
261   /**
262    * Load the snapshotINode field of {@link AbstractINodeDiff}.
263    * @param snapshot The Snapshot associated with the {@link AbstractINodeDiff}.
264    * @param in The {@link DataInput} to read.
265    * @param loader The {@link Loader} instance that this loading procedure is
266    *               using.
267    * @return The snapshotINode.
268    */
loadSnapshotINodeInDirectoryDiff( Snapshot snapshot, DataInput in, FSImageFormat.Loader loader)269   private static INodeDirectoryAttributes loadSnapshotINodeInDirectoryDiff(
270       Snapshot snapshot, DataInput in, FSImageFormat.Loader loader)
271       throws IOException {
272     // read the boolean indicating whether snapshotINode == Snapshot.Root
273     boolean useRoot = in.readBoolean();
274     if (useRoot) {
275       return snapshot.getRoot();
276     } else {
277       // another boolean is used to indicate whether snapshotINode is non-null
278       return in.readBoolean()? loader.loadINodeDirectoryAttributes(in): null;
279     }
280   }
281 
282   /**
283    * Load {@link DirectoryDiff} from fsimage.
284    * @param parent The directory that the SnapshotDiff belongs to.
285    * @param in The {@link DataInput} instance to read.
286    * @param loader The {@link Loader} instance that this loading procedure is
287    *               using.
288    * @return A {@link DirectoryDiff}.
289    */
loadDirectoryDiff(INodeDirectory parent, DataInput in, FSImageFormat.Loader loader)290   private static DirectoryDiff loadDirectoryDiff(INodeDirectory parent,
291       DataInput in, FSImageFormat.Loader loader) throws IOException {
292     // 1. Read the full path of the Snapshot root to identify the Snapshot
293     final Snapshot snapshot = loader.getSnapshot(in);
294 
295     // 2. Load DirectoryDiff#childrenSize
296     int childrenSize = in.readInt();
297 
298     // 3. Load DirectoryDiff#snapshotINode
299     INodeDirectoryAttributes snapshotINode = loadSnapshotINodeInDirectoryDiff(
300         snapshot, in, loader);
301 
302     // 4. Load the created list in SnapshotDiff#Diff
303     List<INode> createdList = loadCreatedList(parent, in);
304 
305     // 5. Load the deleted list in SnapshotDiff#Diff
306     List<INode> deletedList = loadDeletedList(parent, createdList, in, loader);
307 
308     // 6. Compose the SnapshotDiff
309     List<DirectoryDiff> diffs = parent.getDiffs().asList();
310     DirectoryDiff sdiff = new DirectoryDiff(snapshot.getId(), snapshotINode,
311         diffs.isEmpty() ? null : diffs.get(0), childrenSize, createdList,
312         deletedList, snapshotINode == snapshot.getRoot());
313     return sdiff;
314   }
315 
316 
317   /** A reference map for fsimage serialization. */
318   public static class ReferenceMap {
319     /**
320      * Used to indicate whether the reference node itself has been saved
321      */
322     private final Map<Long, INodeReference.WithCount> referenceMap
323         = new HashMap<Long, INodeReference.WithCount>();
324     /**
325      * Used to record whether the subtree of the reference node has been saved
326      */
327     private final Map<Long, Long> dirMap = new HashMap<Long, Long>();
328 
writeINodeReferenceWithCount( INodeReference.WithCount withCount, DataOutput out, boolean writeUnderConstruction)329     public void writeINodeReferenceWithCount(
330         INodeReference.WithCount withCount, DataOutput out,
331         boolean writeUnderConstruction) throws IOException {
332       final INode referred = withCount.getReferredINode();
333       final long id = withCount.getId();
334       final boolean firstReferred = !referenceMap.containsKey(id);
335       out.writeBoolean(firstReferred);
336 
337       if (firstReferred) {
338         FSImageSerialization.saveINode2Image(referred, out,
339             writeUnderConstruction, this);
340         referenceMap.put(id, withCount);
341       } else {
342         out.writeLong(id);
343       }
344     }
345 
toProcessSubtree(long id)346     public boolean toProcessSubtree(long id) {
347       if (dirMap.containsKey(id)) {
348         return false;
349       } else {
350         dirMap.put(id, id);
351         return true;
352       }
353     }
354 
loadINodeReferenceWithCount( boolean isSnapshotINode, DataInput in, FSImageFormat.Loader loader )355     public INodeReference.WithCount loadINodeReferenceWithCount(
356         boolean isSnapshotINode, DataInput in, FSImageFormat.Loader loader
357         ) throws IOException {
358       final boolean firstReferred = in.readBoolean();
359 
360       final INodeReference.WithCount withCount;
361       if (firstReferred) {
362         final INode referred = loader.loadINodeWithLocalName(isSnapshotINode,
363             in, true);
364         withCount = new INodeReference.WithCount(null, referred);
365         referenceMap.put(withCount.getId(), withCount);
366       } else {
367         final long id = in.readLong();
368         withCount = referenceMap.get(id);
369       }
370       return withCount;
371     }
372   }
373 }
374