001/**
002 * Licensed to the Apache Software Foundation (ASF) under one
003 * or more contributor license agreements.  See the NOTICE file
004 * distributed with this work for additional information
005 * regarding copyright ownership.  The ASF licenses this file
006 * to you under the Apache License, Version 2.0 (the
007 * "License"); you may not use this file except in compliance
008 * with the License.  You may obtain a copy of the License at
009 *
010 *     http://www.apache.org/licenses/LICENSE-2.0
011 *
012 * Unless required by applicable law or agreed to in writing, software
013 * distributed under the License is distributed on an "AS IS" BASIS,
014 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
015 * See the License for the specific language governing permissions and
016 * limitations under the License.
017 */
018package org.apache.hadoop.hdfs.server.namenode;
019
020import java.util.Arrays;
021import java.util.Collections;
022import java.util.List;
023import java.util.NoSuchElementException;
024
025import org.apache.commons.logging.Log;
026import org.apache.commons.logging.LogFactory;
027import org.apache.hadoop.hdfs.DFSUtil;
028import org.apache.hadoop.hdfs.protocol.HdfsConstants;
029import org.apache.hadoop.hdfs.server.common.HdfsServerConstants;
030import org.apache.hadoop.hdfs.server.namenode.snapshot.DirectoryWithSnapshotFeature;
031import org.apache.hadoop.hdfs.server.namenode.snapshot.Snapshot;
032
033import com.google.common.base.Preconditions;
034
035import static org.apache.hadoop.hdfs.server.namenode.snapshot.Snapshot.CURRENT_STATE_ID;
036import static org.apache.hadoop.hdfs.server.namenode.snapshot.Snapshot.ID_INTEGER_COMPARATOR;
037
038/**
039 * Contains INodes information resolved from a given path.
040 */
041public class INodesInPath {
042  public static final Log LOG = LogFactory.getLog(INodesInPath.class);
043
044  /**
045   * @return true if path component is {@link HdfsConstants#DOT_SNAPSHOT_DIR}
046   */
047  private static boolean isDotSnapshotDir(byte[] pathComponent) {
048    return pathComponent != null &&
049        Arrays.equals(HdfsServerConstants.DOT_SNAPSHOT_DIR_BYTES, pathComponent);
050  }
051
052  static INodesInPath fromINode(INode inode) {
053    int depth = 0, index;
054    INode tmp = inode;
055    while (tmp != null) {
056      depth++;
057      tmp = tmp.getParent();
058    }
059    final byte[][] path = new byte[depth][];
060    final INode[] inodes = new INode[depth];
061    tmp = inode;
062    index = depth;
063    while (tmp != null) {
064      index--;
065      path[index] = tmp.getKey();
066      inodes[index] = tmp;
067      tmp = tmp.getParent();
068    }
069    return new INodesInPath(inodes, path);
070  }
071
072  static INodesInPath fromComponents(byte[][] components) {
073    return new INodesInPath(new INode[components.length], components);
074  }
075
076  /**
077   * Retrieve existing INodes from a path.  The number of INodes is equal
078   * to the number of path components.  For a snapshot path
079   * (e.g. /foo/.snapshot/s1/bar), the ".snapshot/s1" will be represented in
080   * one path component corresponding to its Snapshot.Root inode.  This 1-1
081   * mapping ensures the path can always be properly reconstructed.
082   *
083   * <p>
084   * Example: <br>
085   * Given the path /c1/c2/c3 where only /c1/c2 exists, resulting in the
086   * following path components: ["","c1","c2","c3"]
087   * 
088   * <p>
089   * <code>getExistingPathINodes(["","c1","c2"])</code> should fill
090   * the array with [rootINode,c1,c2], <br>
091   * <code>getExistingPathINodes(["","c1","c2","c3"])</code> should
092   * fill the array with [rootINode,c1,c2,null]
093   * 
094   * @param startingDir the starting directory
095   * @param components array of path component name
096   * @return the specified number of existing INodes in the path
097   */
098  static INodesInPath resolve(final INodeDirectory startingDir,
099      final byte[][] components) {
100    return resolve(startingDir, components, false);
101  }
102
103  static INodesInPath resolve(final INodeDirectory startingDir,
104      byte[][] components, final boolean isRaw) {
105    Preconditions.checkArgument(startingDir.compareTo(components[0]) == 0);
106
107    INode curNode = startingDir;
108    int count = 0;
109    int inodeNum = 0;
110    INode[] inodes = new INode[components.length];
111    boolean isSnapshot = false;
112    int snapshotId = CURRENT_STATE_ID;
113
114    while (count < components.length && curNode != null) {
115      final boolean lastComp = (count == components.length - 1);
116      inodes[inodeNum++] = curNode;
117      final boolean isRef = curNode.isReference();
118      final boolean isDir = curNode.isDirectory();
119      final INodeDirectory dir = isDir? curNode.asDirectory(): null;
120      if (!isRef && isDir && dir.isWithSnapshot()) {
121        //if the path is a non-snapshot path, update the latest snapshot.
122        if (!isSnapshot && shouldUpdateLatestId(
123            dir.getDirectoryWithSnapshotFeature().getLastSnapshotId(),
124            snapshotId)) {
125          snapshotId = dir.getDirectoryWithSnapshotFeature().getLastSnapshotId();
126        }
127      } else if (isRef && isDir && !lastComp) {
128        // If the curNode is a reference node, need to check its dstSnapshot:
129        // 1. if the existing snapshot is no later than the dstSnapshot (which
130        // is the latest snapshot in dst before the rename), the changes 
131        // should be recorded in previous snapshots (belonging to src).
132        // 2. however, if the ref node is already the last component, we still 
133        // need to know the latest snapshot among the ref node's ancestors, 
134        // in case of processing a deletion operation. Thus we do not overwrite
135        // the latest snapshot if lastComp is true. In case of the operation is
136        // a modification operation, we do a similar check in corresponding 
137        // recordModification method.
138        if (!isSnapshot) {
139          int dstSnapshotId = curNode.asReference().getDstSnapshotId();
140          if (snapshotId == CURRENT_STATE_ID || // no snapshot in dst tree of rename
141              (dstSnapshotId != CURRENT_STATE_ID &&
142               dstSnapshotId >= snapshotId)) { // the above scenario
143            int lastSnapshot = CURRENT_STATE_ID;
144            DirectoryWithSnapshotFeature sf;
145            if (curNode.isDirectory() && 
146                (sf = curNode.asDirectory().getDirectoryWithSnapshotFeature()) != null) {
147              lastSnapshot = sf.getLastSnapshotId();
148            }
149            snapshotId = lastSnapshot;
150          }
151        }
152      }
153      if (lastComp || !isDir) {
154        break;
155      }
156
157      final byte[] childName = components[++count];
158      // check if the next byte[] in components is for ".snapshot"
159      if (isDotSnapshotDir(childName) && dir.isSnapshottable()) {
160        isSnapshot = true;
161        // check if ".snapshot" is the last element of components
162        if (count == components.length - 1) {
163          break;
164        }
165        // Resolve snapshot root
166        final Snapshot s = dir.getSnapshot(components[count + 1]);
167        if (s == null) {
168          curNode = null; // snapshot not found
169        } else {
170          curNode = s.getRoot();
171          snapshotId = s.getId();
172        }
173        // combine .snapshot & name into 1 component element to ensure
174        // 1-to-1 correspondence between components and inodes arrays is
175        // preserved so a path can be reconstructed.
176        byte[][] componentsCopy =
177            Arrays.copyOf(components, components.length - 1);
178        componentsCopy[count] = DFSUtil.string2Bytes(
179            DFSUtil.byteArray2PathString(components, count, 2));
180        // shift the remaining components after snapshot name
181        int start = count + 2;
182        System.arraycopy(components, start, componentsCopy, count + 1,
183            components.length - start);
184        components = componentsCopy;
185        // reduce the inodes array to compensate for reduction in components
186        inodes = Arrays.copyOf(inodes, components.length);
187      } else {
188        // normal case, and also for resolving file/dir under snapshot root
189        curNode = dir.getChild(childName,
190            isSnapshot ? snapshotId : CURRENT_STATE_ID);
191      }
192    }
193    return new INodesInPath(inodes, components, isRaw, isSnapshot, snapshotId);
194  }
195
196  private static boolean shouldUpdateLatestId(int sid, int snapshotId) {
197    return snapshotId == CURRENT_STATE_ID || (sid != CURRENT_STATE_ID &&
198        ID_INTEGER_COMPARATOR.compare(snapshotId, sid) < 0);
199  }
200
201  /**
202   * Replace an inode of the given INodesInPath in the given position. We do a
203   * deep copy of the INode array.
204   * @param pos the position of the replacement
205   * @param inode the new inode
206   * @return a new INodesInPath instance
207   */
208  public static INodesInPath replace(INodesInPath iip, int pos, INode inode) {
209    Preconditions.checkArgument(iip.length() > 0 && pos > 0 // no for root
210        && pos < iip.length());
211    if (iip.getINode(pos) == null) {
212      Preconditions.checkState(iip.getINode(pos - 1) != null);
213    }
214    INode[] inodes = new INode[iip.inodes.length];
215    System.arraycopy(iip.inodes, 0, inodes, 0, inodes.length);
216    inodes[pos] = inode;
217    return new INodesInPath(inodes, iip.path, iip.isRaw,
218        iip.isSnapshot, iip.snapshotId);
219  }
220
221  /**
222   * Extend a given INodesInPath with a child INode. The child INode will be
223   * appended to the end of the new INodesInPath.
224   */
225  public static INodesInPath append(INodesInPath iip, INode child,
226      byte[] childName) {
227    Preconditions.checkArgument(iip.length() > 0);
228    Preconditions.checkArgument(iip.getLastINode() != null && iip
229        .getLastINode().isDirectory());
230    INode[] inodes = new INode[iip.length() + 1];
231    System.arraycopy(iip.inodes, 0, inodes, 0, inodes.length - 1);
232    inodes[inodes.length - 1] = child;
233    byte[][] path = new byte[iip.path.length + 1][];
234    System.arraycopy(iip.path, 0, path, 0, path.length - 1);
235    path[path.length - 1] = childName;
236    return new INodesInPath(inodes, path, iip.isRaw,
237        iip.isSnapshot, iip.snapshotId);
238  }
239
240  private final byte[][] path;
241  private final String pathname;
242
243  /**
244   * Array with the specified number of INodes resolved for a given path.
245   */
246  private final INode[] inodes;
247  /**
248   * true if this path corresponds to a snapshot
249   */
250  private final boolean isSnapshot;
251
252  /**
253   * true if this is a /.reserved/raw path.  path component resolution strips
254   * it from the path so need to track it separately.
255   */
256  private final boolean isRaw;
257
258  /**
259   * For snapshot paths, it is the id of the snapshot; or 
260   * {@link Snapshot#CURRENT_STATE_ID} if the snapshot does not exist. For 
261   * non-snapshot paths, it is the id of the latest snapshot found in the path;
262   * or {@link Snapshot#CURRENT_STATE_ID} if no snapshot is found.
263   */
264  private final int snapshotId;
265
266  private INodesInPath(INode[] inodes, byte[][] path, boolean isRaw,
267      boolean isSnapshot,int snapshotId) {
268    Preconditions.checkArgument(inodes != null && path != null);
269    this.inodes = inodes;
270    this.path = path;
271    this.pathname = DFSUtil.byteArray2PathString(path);
272    this.isRaw = isRaw;
273    this.isSnapshot = isSnapshot;
274    this.snapshotId = snapshotId;
275  }
276
277  private INodesInPath(INode[] inodes, byte[][] path) {
278    this(inodes, path, false, false, CURRENT_STATE_ID);
279  }
280
281  /**
282   * For non-snapshot paths, return the latest snapshot id found in the path.
283   */
284  public int getLatestSnapshotId() {
285    Preconditions.checkState(!isSnapshot);
286    return snapshotId;
287  }
288  
289  /**
290   * For snapshot paths, return the id of the snapshot specified in the path.
291   * For non-snapshot paths, return {@link Snapshot#CURRENT_STATE_ID}.
292   */
293  public int getPathSnapshotId() {
294    return isSnapshot ? snapshotId : CURRENT_STATE_ID;
295  }
296
297  /**
298   * @return the i-th inode if i >= 0;
299   *         otherwise, i < 0, return the (length + i)-th inode.
300   */
301  public INode getINode(int i) {
302    if (inodes == null || inodes.length == 0) {
303      throw new NoSuchElementException("inodes is null or empty");
304    }
305    int index = i >= 0 ? i : inodes.length + i;
306    if (index < inodes.length && index >= 0) {
307      return inodes[index];
308    } else {
309      throw new NoSuchElementException("inodes.length == " + inodes.length);
310    }
311  }
312  
313  /** @return the last inode. */
314  public INode getLastINode() {
315    return getINode(-1);
316  }
317
318  byte[] getLastLocalName() {
319    return path[path.length - 1];
320  }
321
322  public byte[][] getPathComponents() {
323    return path;
324  }
325
326  public byte[] getPathComponent(int i) {
327    return path[i];
328  }
329
330  /** @return the full path in string form */
331  public String getPath() {
332    return pathname;
333  }
334
335  public String getParentPath() {
336    return getPath(path.length - 2);
337  }
338
339  public String getPath(int pos) {
340    return DFSUtil.byteArray2PathString(path, 0, pos + 1); // it's a length...
341  }
342
343  public int length() {
344    return inodes.length;
345  }
346
347  public List<INode> getReadOnlyINodes() {
348    return Collections.unmodifiableList(Arrays.asList(inodes));
349  }
350
351  public INode[] getINodesArray() {
352    INode[] retArr = new INode[inodes.length];
353    System.arraycopy(inodes, 0, retArr, 0, inodes.length);
354    return retArr;
355  }
356
357  /**
358   * @param length number of ancestral INodes in the returned INodesInPath
359   *               instance
360   * @return the INodesInPath instance containing ancestral INodes. Note that
361   * this method only handles non-snapshot paths.
362   */
363  private INodesInPath getAncestorINodesInPath(int length) {
364    Preconditions.checkArgument(length >= 0 && length < inodes.length);
365    Preconditions.checkState(isDotSnapshotDir() || !isSnapshot());
366    final INode[] anodes = new INode[length];
367    final byte[][] apath = new byte[length][];
368    System.arraycopy(this.inodes, 0, anodes, 0, length);
369    System.arraycopy(this.path, 0, apath, 0, length);
370    return new INodesInPath(anodes, apath, isRaw, false, snapshotId);
371  }
372
373  /**
374   * @return an INodesInPath instance containing all the INodes in the parent
375   *         path. We do a deep copy here.
376   */
377  public INodesInPath getParentINodesInPath() {
378    return inodes.length > 1 ? getAncestorINodesInPath(inodes.length - 1) :
379        null;
380  }
381
382  /**
383   * @return a new INodesInPath instance that only contains existing INodes.
384   * Note that this method only handles non-snapshot paths.
385   */
386  public INodesInPath getExistingINodes() {
387    Preconditions.checkState(!isSnapshot());
388    for (int i = inodes.length; i > 0; i--) {
389      if (inodes[i - 1] != null) {
390        return (i == inodes.length) ? this : getAncestorINodesInPath(i);
391      }
392    }
393    return null;
394  }
395
396  /**
397   * @return isSnapshot true for a snapshot path
398   */
399  boolean isSnapshot() {
400    return this.isSnapshot;
401  }
402
403  /**
404   * @return if .snapshot is the last path component.
405   */
406  boolean isDotSnapshotDir() {
407    return isDotSnapshotDir(getLastLocalName());
408  }
409
410  /**
411   * @return if this is a /.reserved/raw path.
412   */
413  public boolean isRaw() {
414    return isRaw;
415  }
416
417  private static String toString(INode inode) {
418    return inode == null? null: inode.getLocalName();
419  }
420
421  @Override
422  public String toString() {
423    return toString(true);
424  }
425
426  private String toString(boolean vaildateObject) {
427    if (vaildateObject) {
428      validate();
429    }
430
431    final StringBuilder b = new StringBuilder(getClass().getSimpleName())
432        .append(": path = ").append(DFSUtil.byteArray2PathString(path))
433        .append("\n  inodes = ");
434    if (inodes == null) {
435      b.append("null");
436    } else if (inodes.length == 0) {
437      b.append("[]");
438    } else {
439      b.append("[").append(toString(inodes[0]));
440      for(int i = 1; i < inodes.length; i++) {
441        b.append(", ").append(toString(inodes[i]));
442      }
443      b.append("], length=").append(inodes.length);
444    }
445    b.append("\n  isSnapshot        = ").append(isSnapshot)
446     .append("\n  snapshotId        = ").append(snapshotId);
447    return b.toString();
448  }
449
450  void validate() {
451    // check parent up to snapshotRootIndex if this is a snapshot path
452    int i = 0;
453    if (inodes[i] != null) {
454      for(i++; i < inodes.length && inodes[i] != null; i++) {
455        final INodeDirectory parent_i = inodes[i].getParent();
456        final INodeDirectory parent_i_1 = inodes[i-1].getParent();
457        if (parent_i != inodes[i-1] &&
458            (parent_i_1 == null || !parent_i_1.isSnapshottable()
459                || parent_i != parent_i_1)) {
460          throw new AssertionError(
461              "inodes[" + i + "].getParent() != inodes[" + (i-1)
462              + "]\n  inodes[" + i + "]=" + inodes[i].toDetailString()
463              + "\n  inodes[" + (i-1) + "]=" + inodes[i-1].toDetailString()
464              + "\n this=" + toString(false));
465        }
466      }
467    }
468    if (i != inodes.length) {
469      throw new AssertionError("i = " + i + " != " + inodes.length
470          + ", this=" + toString(false));
471    }
472  }
473}