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