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