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 com.google.common.collect.ImmutableList;
026import org.apache.commons.logging.Log;
027import org.apache.commons.logging.LogFactory;
028import org.apache.hadoop.fs.Path;
029import org.apache.hadoop.fs.UnresolvedLinkException;
030import org.apache.hadoop.hdfs.DFSUtil;
031import org.apache.hadoop.hdfs.protocol.HdfsConstants;
032import org.apache.hadoop.hdfs.protocol.UnresolvedPathException;
033import org.apache.hadoop.hdfs.server.namenode.snapshot.DirectoryWithSnapshotFeature;
034import org.apache.hadoop.hdfs.server.namenode.snapshot.Snapshot;
035
036import com.google.common.base.Preconditions;
037
038import static org.apache.hadoop.hdfs.server.namenode.snapshot.Snapshot.CURRENT_STATE_ID;
039import static org.apache.hadoop.hdfs.server.namenode.snapshot.Snapshot.ID_INTEGER_COMPARATOR;
040
041/**
042 * Contains INodes information resolved from a given path.
043 */
044public class INodesInPath {
045  public static final Log LOG = LogFactory.getLog(INodesInPath.class);
046
047  /**
048   * @return true if path component is {@link HdfsConstants#DOT_SNAPSHOT_DIR}
049   */
050  private static boolean isDotSnapshotDir(byte[] pathComponent) {
051    return pathComponent != null &&
052        Arrays.equals(HdfsConstants.DOT_SNAPSHOT_DIR_BYTES, pathComponent);
053  }
054
055  static INodesInPath fromINode(INode inode) {
056    int depth = 0, index;
057    INode tmp = inode;
058    while (tmp != null) {
059      depth++;
060      tmp = tmp.getParent();
061    }
062    final byte[][] path = new byte[depth][];
063    final INode[] inodes = new INode[depth];
064    tmp = inode;
065    index = depth;
066    while (tmp != null) {
067      index--;
068      path[index] = tmp.getKey();
069      inodes[index] = tmp;
070      tmp = tmp.getParent();
071    }
072    return new INodesInPath(inodes, path);
073  }
074
075  static INodesInPath fromComponents(byte[][] components) {
076    return new INodesInPath(new INode[components.length], components);
077  }
078
079  /**
080   * Given some components, create a path name.
081   * @param components The path components
082   * @param start index
083   * @param end index
084   * @return concatenated path
085   */
086  private static String constructPath(byte[][] components, int start, int end) {
087    StringBuilder buf = new StringBuilder();
088    for (int i = start; i < end; i++) {
089      buf.append(DFSUtil.bytes2String(components[i]));
090      if (i < end - 1) {
091        buf.append(Path.SEPARATOR);
092      }
093    }
094    return buf.toString();
095  }
096
097  /**
098   * Retrieve existing INodes from a path. For non-snapshot path,
099   * the number of INodes is equal to the number of path components. For
100   * snapshot path (e.g., /foo/.snapshot/s1/bar), the number of INodes is
101   * (number_of_path_components - 1).
102   * 
103   * An UnresolvedPathException is always thrown when an intermediate path 
104   * component refers to a symbolic link. If the final path component refers 
105   * to a symbolic link then an UnresolvedPathException is only thrown if
106   * resolveLink is true.  
107   * 
108   * <p>
109   * Example: <br>
110   * Given the path /c1/c2/c3 where only /c1/c2 exists, resulting in the
111   * following path components: ["","c1","c2","c3"]
112   * 
113   * <p>
114   * <code>getExistingPathINodes(["","c1","c2"])</code> should fill
115   * the array with [rootINode,c1,c2], <br>
116   * <code>getExistingPathINodes(["","c1","c2","c3"])</code> should
117   * fill the array with [rootINode,c1,c2,null]
118   * 
119   * @param startingDir the starting directory
120   * @param components array of path component name
121   * @param resolveLink indicates whether UnresolvedLinkException should
122   *        be thrown when the path refers to a symbolic link.
123   * @return the specified number of existing INodes in the path
124   */
125  static INodesInPath resolve(final INodeDirectory startingDir,
126      final byte[][] components, final boolean resolveLink)
127      throws UnresolvedLinkException {
128    Preconditions.checkArgument(startingDir.compareTo(components[0]) == 0);
129
130    INode curNode = startingDir;
131    int count = 0;
132    int inodeNum = 0;
133    INode[] inodes = new INode[components.length];
134    boolean isSnapshot = false;
135    int snapshotId = CURRENT_STATE_ID;
136
137    while (count < components.length && curNode != null) {
138      final boolean lastComp = (count == components.length - 1);
139      inodes[inodeNum++] = curNode;
140      final boolean isRef = curNode.isReference();
141      final boolean isDir = curNode.isDirectory();
142      final INodeDirectory dir = isDir? curNode.asDirectory(): null;
143      if (!isRef && isDir && dir.isWithSnapshot()) {
144        //if the path is a non-snapshot path, update the latest snapshot.
145        if (!isSnapshot && shouldUpdateLatestId(
146            dir.getDirectoryWithSnapshotFeature().getLastSnapshotId(),
147            snapshotId)) {
148          snapshotId = dir.getDirectoryWithSnapshotFeature().getLastSnapshotId();
149        }
150      } else if (isRef && isDir && !lastComp) {
151        // If the curNode is a reference node, need to check its dstSnapshot:
152        // 1. if the existing snapshot is no later than the dstSnapshot (which
153        // is the latest snapshot in dst before the rename), the changes 
154        // should be recorded in previous snapshots (belonging to src).
155        // 2. however, if the ref node is already the last component, we still 
156        // need to know the latest snapshot among the ref node's ancestors, 
157        // in case of processing a deletion operation. Thus we do not overwrite
158        // the latest snapshot if lastComp is true. In case of the operation is
159        // a modification operation, we do a similar check in corresponding 
160        // recordModification method.
161        if (!isSnapshot) {
162          int dstSnapshotId = curNode.asReference().getDstSnapshotId();
163          if (snapshotId == CURRENT_STATE_ID || // no snapshot in dst tree of rename
164              (dstSnapshotId != CURRENT_STATE_ID &&
165               dstSnapshotId >= snapshotId)) { // the above scenario
166            int lastSnapshot = CURRENT_STATE_ID;
167            DirectoryWithSnapshotFeature sf;
168            if (curNode.isDirectory() && 
169                (sf = curNode.asDirectory().getDirectoryWithSnapshotFeature()) != null) {
170              lastSnapshot = sf.getLastSnapshotId();
171            }
172            snapshotId = lastSnapshot;
173          }
174        }
175      }
176      if (curNode.isSymlink() && (!lastComp || resolveLink)) {
177        final String path = constructPath(components, 0, components.length);
178        final String preceding = constructPath(components, 0, count);
179        final String remainder =
180          constructPath(components, count + 1, components.length);
181        final String link = DFSUtil.bytes2String(components[count]);
182        final String target = curNode.asSymlink().getSymlinkString();
183        if (LOG.isDebugEnabled()) {
184          LOG.debug("UnresolvedPathException " +
185            " path: " + path + " preceding: " + preceding +
186            " count: " + count + " link: " + link + " target: " + target +
187            " remainder: " + remainder);
188        }
189        throw new UnresolvedPathException(path, preceding, remainder, target);
190      }
191      if (lastComp || !isDir) {
192        break;
193      }
194      final byte[] childName = components[count + 1];
195      
196      // check if the next byte[] in components is for ".snapshot"
197      if (isDotSnapshotDir(childName) && dir.isSnapshottable()) {
198        // skip the ".snapshot" in components
199        count++;
200        isSnapshot = true;
201        // check if ".snapshot" is the last element of components
202        if (count == components.length - 1) {
203          break;
204        }
205        // Resolve snapshot root
206        final Snapshot s = dir.getSnapshot(components[count + 1]);
207        if (s == null) {
208          curNode = null; // snapshot not found
209        } else {
210          curNode = s.getRoot();
211          snapshotId = s.getId();
212        }
213      } else {
214        // normal case, and also for resolving file/dir under snapshot root
215        curNode = dir.getChild(childName,
216            isSnapshot ? snapshotId : CURRENT_STATE_ID);
217      }
218      count++;
219    }
220    if (isSnapshot && !isDotSnapshotDir(components[components.length - 1])) {
221      // for snapshot path shrink the inode array. however, for path ending with
222      // .snapshot, still keep last the null inode in the array
223      INode[] newNodes = new INode[components.length - 1];
224      System.arraycopy(inodes, 0, newNodes, 0, newNodes.length);
225      inodes = newNodes;
226    }
227    return new INodesInPath(inodes, components, isSnapshot, snapshotId);
228  }
229
230  private static boolean shouldUpdateLatestId(int sid, int snapshotId) {
231    return snapshotId == CURRENT_STATE_ID || (sid != CURRENT_STATE_ID &&
232        ID_INTEGER_COMPARATOR.compare(snapshotId, sid) < 0);
233  }
234
235  /**
236   * Replace an inode of the given INodesInPath in the given position. We do a
237   * deep copy of the INode array.
238   * @param pos the position of the replacement
239   * @param inode the new inode
240   * @return a new INodesInPath instance
241   */
242  public static INodesInPath replace(INodesInPath iip, int pos, INode inode) {
243    Preconditions.checkArgument(iip.length() > 0 && pos > 0 // no for root
244        && pos < iip.length());
245    if (iip.getINode(pos) == null) {
246      Preconditions.checkState(iip.getINode(pos - 1) != null);
247    }
248    INode[] inodes = new INode[iip.inodes.length];
249    System.arraycopy(iip.inodes, 0, inodes, 0, inodes.length);
250    inodes[pos] = inode;
251    return new INodesInPath(inodes, iip.path, iip.isSnapshot, iip.snapshotId);
252  }
253
254  /**
255   * Extend a given INodesInPath with a child INode. The child INode will be
256   * appended to the end of the new INodesInPath.
257   */
258  public static INodesInPath append(INodesInPath iip, INode child,
259      byte[] childName) {
260    Preconditions.checkArgument(iip.length() > 0);
261    Preconditions.checkArgument(iip.getLastINode() != null && iip
262        .getLastINode().isDirectory());
263    INode[] inodes = new INode[iip.length() + 1];
264    System.arraycopy(iip.inodes, 0, inodes, 0, inodes.length - 1);
265    inodes[inodes.length - 1] = child;
266    byte[][] path = new byte[iip.path.length + 1][];
267    System.arraycopy(iip.path, 0, path, 0, path.length - 1);
268    path[path.length - 1] = childName;
269    return new INodesInPath(inodes, path, iip.isSnapshot, iip.snapshotId);
270  }
271
272  private final byte[][] path;
273  private volatile String pathname;
274
275  /**
276   * Array with the specified number of INodes resolved for a given path.
277   */
278  private final INode[] inodes;
279  /**
280   * true if this path corresponds to a snapshot
281   */
282  private final boolean isSnapshot;
283  /**
284   * For snapshot paths, it is the id of the snapshot; or 
285   * {@link Snapshot#CURRENT_STATE_ID} if the snapshot does not exist. For 
286   * non-snapshot paths, it is the id of the latest snapshot found in the path;
287   * or {@link Snapshot#CURRENT_STATE_ID} if no snapshot is found.
288   */
289  private final int snapshotId;
290
291  private INodesInPath(INode[] inodes, byte[][] path, boolean isSnapshot,
292      int snapshotId) {
293    Preconditions.checkArgument(inodes != null && path != null);
294    this.inodes = inodes;
295    this.path = path;
296    this.isSnapshot = isSnapshot;
297    this.snapshotId = snapshotId;
298  }
299
300  private INodesInPath(INode[] inodes, byte[][] path) {
301    this(inodes, path, false, CURRENT_STATE_ID);
302  }
303
304  /**
305   * For non-snapshot paths, return the latest snapshot id found in the path.
306   */
307  public int getLatestSnapshotId() {
308    Preconditions.checkState(!isSnapshot);
309    return snapshotId;
310  }
311  
312  /**
313   * For snapshot paths, return the id of the snapshot specified in the path.
314   * For non-snapshot paths, return {@link Snapshot#CURRENT_STATE_ID}.
315   */
316  public int getPathSnapshotId() {
317    return isSnapshot ? snapshotId : CURRENT_STATE_ID;
318  }
319
320  /**
321   * @return the i-th inode if i >= 0;
322   *         otherwise, i < 0, return the (length + i)-th inode.
323   */
324  public INode getINode(int i) {
325    if (inodes == null || inodes.length == 0) {
326      throw new NoSuchElementException("inodes is null or empty");
327    }
328    int index = i >= 0 ? i : inodes.length + i;
329    if (index < inodes.length && index >= 0) {
330      return inodes[index];
331    } else {
332      throw new NoSuchElementException("inodes.length == " + inodes.length);
333    }
334  }
335  
336  /** @return the last inode. */
337  public INode getLastINode() {
338    return getINode(-1);
339  }
340
341  byte[] getLastLocalName() {
342    return path[path.length - 1];
343  }
344
345  public byte[][] getPathComponents() {
346    return path;
347  }
348
349  /** @return the full path in string form */
350  public String getPath() {
351    if (pathname == null) {
352      pathname = DFSUtil.byteArray2PathString(path);
353    }
354    return pathname;
355  }
356
357  public String getParentPath() {
358    return getPath(path.length - 2);
359  }
360
361  public String getPath(int pos) {
362    return DFSUtil.byteArray2PathString(path, 0, pos + 1); // it's a length...
363  }
364
365  /**
366   * @param offset start endpoint (inclusive)
367   * @param length number of path components
368   * @return sub-list of the path
369   */
370  public List<String> getPath(int offset, int length) {
371    Preconditions.checkArgument(offset >= 0 && length >= 0 && offset + length
372        <= path.length);
373    ImmutableList.Builder<String> components = ImmutableList.builder();
374    for (int i = offset; i < offset + length; i++) {
375      components.add(DFSUtil.bytes2String(path[i]));
376    }
377    return components.build();
378  }
379
380  public int length() {
381    return inodes.length;
382  }
383
384  public List<INode> getReadOnlyINodes() {
385    return Collections.unmodifiableList(Arrays.asList(inodes));
386  }
387
388  public INode[] getINodesArray() {
389    INode[] retArr = new INode[inodes.length];
390    System.arraycopy(inodes, 0, retArr, 0, inodes.length);
391    return retArr;
392  }
393
394  /**
395   * @param length number of ancestral INodes in the returned INodesInPath
396   *               instance
397   * @return the INodesInPath instance containing ancestral INodes. Note that
398   * this method only handles non-snapshot paths.
399   */
400  private INodesInPath getAncestorINodesInPath(int length) {
401    Preconditions.checkArgument(length >= 0 && length < inodes.length);
402    Preconditions.checkState(!isSnapshot());
403    final INode[] anodes = new INode[length];
404    final byte[][] apath = new byte[length][];
405    System.arraycopy(this.inodes, 0, anodes, 0, length);
406    System.arraycopy(this.path, 0, apath, 0, length);
407    return new INodesInPath(anodes, apath, false, snapshotId);
408  }
409
410  /**
411   * @return an INodesInPath instance containing all the INodes in the parent
412   *         path. We do a deep copy here.
413   */
414  public INodesInPath getParentINodesInPath() {
415    return inodes.length > 1 ? getAncestorINodesInPath(inodes.length - 1) :
416        null;
417  }
418
419  /**
420   * @return a new INodesInPath instance that only contains exisitng INodes.
421   * Note that this method only handles non-snapshot paths.
422   */
423  public INodesInPath getExistingINodes() {
424    Preconditions.checkState(!isSnapshot());
425    int i = 0;
426    for (; i < inodes.length; i++) {
427      if (inodes[i] == null) {
428        break;
429      }
430    }
431    INode[] existing = new INode[i];
432    byte[][] existingPath = new byte[i][];
433    System.arraycopy(inodes, 0, existing, 0, i);
434    System.arraycopy(path, 0, existingPath, 0, i);
435    return new INodesInPath(existing, existingPath, false, snapshotId);
436  }
437
438  /**
439   * @return isSnapshot true for a snapshot path
440   */
441  boolean isSnapshot() {
442    return this.isSnapshot;
443  }
444
445  boolean isDotSnapshotDir() {
446    return isDotSnapshotDir(getLastLocalName());
447  }
448
449  private static String toString(INode inode) {
450    return inode == null? null: inode.getLocalName();
451  }
452
453  @Override
454  public String toString() {
455    return toString(true);
456  }
457
458  private String toString(boolean vaildateObject) {
459    if (vaildateObject) {
460      validate();
461    }
462
463    final StringBuilder b = new StringBuilder(getClass().getSimpleName())
464        .append(": path = ").append(DFSUtil.byteArray2PathString(path))
465        .append("\n  inodes = ");
466    if (inodes == null) {
467      b.append("null");
468    } else if (inodes.length == 0) {
469      b.append("[]");
470    } else {
471      b.append("[").append(toString(inodes[0]));
472      for(int i = 1; i < inodes.length; i++) {
473        b.append(", ").append(toString(inodes[i]));
474      }
475      b.append("], length=").append(inodes.length);
476    }
477    b.append("\n  isSnapshot        = ").append(isSnapshot)
478     .append("\n  snapshotId        = ").append(snapshotId);
479    return b.toString();
480  }
481
482  void validate() {
483    // check parent up to snapshotRootIndex if this is a snapshot path
484    int i = 0;
485    if (inodes[i] != null) {
486      for(i++; i < inodes.length && inodes[i] != null; i++) {
487        final INodeDirectory parent_i = inodes[i].getParent();
488        final INodeDirectory parent_i_1 = inodes[i-1].getParent();
489        if (parent_i != inodes[i-1] &&
490            (parent_i_1 == null || !parent_i_1.isSnapshottable()
491                || parent_i != parent_i_1)) {
492          throw new AssertionError(
493              "inodes[" + i + "].getParent() != inodes[" + (i-1)
494              + "]\n  inodes[" + i + "]=" + inodes[i].toDetailString()
495              + "\n  inodes[" + (i-1) + "]=" + inodes[i-1].toDetailString()
496              + "\n this=" + toString(false));
497        }
498      }
499    }
500    if (i != inodes.length) {
501      throw new AssertionError("i = " + i + " != " + inodes.length
502          + ", this=" + toString(false));
503    }
504  }
505}