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.fs;
019
020import java.io.FileNotFoundException;
021import java.io.IOException;
022import java.io.InputStream;
023import java.io.OutputStream;
024import java.net.URI;
025import java.security.PrivilegedExceptionAction;
026import java.util.ArrayList;
027import java.util.Collection;
028import java.util.EnumSet;
029import java.util.HashSet;
030import java.util.IdentityHashMap;
031import java.util.List;
032import java.util.Map;
033import java.util.Set;
034import java.util.Stack;
035import java.util.TreeSet;
036import java.util.Map.Entry;
037
038import org.apache.commons.logging.Log;
039import org.apache.commons.logging.LogFactory;
040import org.apache.hadoop.HadoopIllegalArgumentException;
041import org.apache.hadoop.classification.InterfaceAudience;
042import org.apache.hadoop.classification.InterfaceStability;
043import org.apache.hadoop.conf.Configuration;
044import org.apache.hadoop.fs.FileSystem.Statistics;
045import org.apache.hadoop.fs.Options.CreateOpts;
046import org.apache.hadoop.fs.permission.AclEntry;
047import org.apache.hadoop.fs.permission.AclStatus;
048import org.apache.hadoop.fs.permission.FsAction;
049import org.apache.hadoop.fs.permission.FsPermission;
050import static org.apache.hadoop.fs.CommonConfigurationKeysPublic.FS_DEFAULT_NAME_KEY;
051import static org.apache.hadoop.fs.CommonConfigurationKeysPublic.FS_DEFAULT_NAME_DEFAULT;
052
053import org.apache.hadoop.io.IOUtils;
054import org.apache.hadoop.ipc.RpcClientException;
055import org.apache.hadoop.ipc.RpcServerException;
056import org.apache.hadoop.ipc.UnexpectedServerException;
057import org.apache.hadoop.fs.InvalidPathException;
058import org.apache.hadoop.security.AccessControlException;
059import org.apache.hadoop.security.UserGroupInformation;
060import org.apache.hadoop.security.token.Token;
061import org.apache.hadoop.util.ShutdownHookManager;
062
063import com.google.common.base.Preconditions;
064import org.apache.htrace.core.Tracer;
065
066/**
067 * The FileContext class provides an interface for users of the Hadoop
068 * file system. It exposes a number of file system operations, e.g. create,
069 * open, list.
070 * 
071 * <h2>Path Names</h2>
072 * 
073 * The Hadoop file system supports a URI namespace and URI names. This enables
074 * multiple types of file systems to be referenced using fully-qualified URIs.
075 * Two common Hadoop file system implementations are
076 * <ul>
077 * <li>the local file system: file:///path
078 * <li>the HDFS file system: hdfs://nnAddress:nnPort/path
079 * </ul>
080 * 
081 * The Hadoop file system also supports additional naming schemes besides URIs.
082 * Hadoop has the concept of a <i>default file system</i>, which implies a
083 * default URI scheme and authority. This enables <i>slash-relative names</i>
084 * relative to the default FS, which are more convenient for users and
085 * application writers. The default FS is typically set by the user's
086 * environment, though it can also be manually specified.
087 * <p>
088 * 
089 * Hadoop also supports <i>working-directory-relative</i> names, which are paths
090 * relative to the current working directory (similar to Unix). The working
091 * directory can be in a different file system than the default FS.
092 * <p>
093 * Thus, Hadoop path names can be specified as one of the following:
094 * <ul>
095 * <li>a fully-qualified URI: scheme://authority/path (e.g.
096 * hdfs://nnAddress:nnPort/foo/bar)
097 * <li>a slash-relative name: path relative to the default file system (e.g.
098 * /foo/bar)
099 * <li>a working-directory-relative name: path relative to the working dir (e.g.
100 * foo/bar)
101 * </ul>
102 *  Relative paths with scheme (scheme:foo/bar) are illegal.
103 *  
104 * <h2>Role of FileContext and Configuration Defaults</h2>
105 *
106 * The FileContext is the analogue of per-process file-related state in Unix. It
107 * contains two properties:
108 * 
109 * <ul>
110 * <li>the default file system (for resolving slash-relative names)
111 * <li>the umask (for file permissions)
112 * </ul>
113 * In general, these properties are obtained from the default configuration file
114 * in the user's environment (see {@link Configuration}).
115 * 
116 * Further file system properties are specified on the server-side. File system
117 * operations default to using these server-side defaults unless otherwise
118 * specified.
119 * <p>
120 * The file system related server-side defaults are:
121 *  <ul>
122 *  <li> the home directory (default is "/user/userName")
123 *  <li> the initial wd (only for local fs)
124 *  <li> replication factor
125 *  <li> block size
126 *  <li> buffer size
127 *  <li> encryptDataTransfer 
128 *  <li> checksum option. (checksumType and  bytesPerChecksum)
129 *  </ul>
130 *
131 * <h2>Example Usage</h2>
132 *
133 * Example 1: use the default config read from the $HADOOP_CONFIG/core.xml.
134 *   Unspecified values come from core-defaults.xml in the release jar.
135 *  <ul>  
136 *  <li> myFContext = FileContext.getFileContext(); // uses the default config
137 *                                                // which has your default FS 
138 *  <li>  myFContext.create(path, ...);
139 *  <li>  myFContext.setWorkingDir(path);
140 *  <li>  myFContext.open (path, ...);  
141 *  <li>...
142 *  </ul>  
143 * Example 2: Get a FileContext with a specific URI as the default FS
144 *  <ul>  
145 *  <li> myFContext = FileContext.getFileContext(URI);
146 *  <li> myFContext.create(path, ...);
147 *  <li>...
148 * </ul>
149 * Example 3: FileContext with local file system as the default
150 *  <ul> 
151 *  <li> myFContext = FileContext.getLocalFSFileContext();
152 *  <li> myFContext.create(path, ...);
153 *  <li> ...
154 *  </ul> 
155 * Example 4: Use a specific config, ignoring $HADOOP_CONFIG
156 *  Generally you should not need use a config unless you are doing
157 *   <ul> 
158 *   <li> configX = someConfigSomeOnePassedToYou;
159 *   <li> myFContext = getFileContext(configX); // configX is not changed,
160 *                                              // is passed down 
161 *   <li> myFContext.create(path, ...);
162 *   <li>...
163 *  </ul>                                          
164 *    
165 */
166
167@InterfaceAudience.Public
168@InterfaceStability.Stable
169public class FileContext {
170  
171  public static final Log LOG = LogFactory.getLog(FileContext.class);
172  /**
173   * Default permission for directory and symlink
174   * In previous versions, this default permission was also used to
175   * create files, so files created end up with ugo+x permission.
176   * See HADOOP-9155 for detail. 
177   * Two new constants are added to solve this, please use 
178   * {@link FileContext#DIR_DEFAULT_PERM} for directory, and use
179   * {@link FileContext#FILE_DEFAULT_PERM} for file.
180   * This constant is kept for compatibility.
181   */
182  public static final FsPermission DEFAULT_PERM = FsPermission.getDefault();
183  /**
184   * Default permission for directory
185   */
186  public static final FsPermission DIR_DEFAULT_PERM = FsPermission.getDirDefault();
187  /**
188   * Default permission for file
189   */
190  public static final FsPermission FILE_DEFAULT_PERM = FsPermission.getFileDefault();
191
192  /**
193   * Priority of the FileContext shutdown hook.
194   */
195  public static final int SHUTDOWN_HOOK_PRIORITY = 20;
196
197  /**
198   * List of files that should be deleted on JVM shutdown.
199   */
200  static final Map<FileContext, Set<Path>> DELETE_ON_EXIT = 
201    new IdentityHashMap<FileContext, Set<Path>>();
202
203  /** JVM shutdown hook thread. */
204  static final FileContextFinalizer FINALIZER = 
205    new FileContextFinalizer();
206  
207  private static final PathFilter DEFAULT_FILTER = new PathFilter() {
208    @Override
209    public boolean accept(final Path file) {
210      return true;
211    }
212  };
213  
214  /**
215   * The FileContext is defined by.
216   *  1) defaultFS (slash)
217   *  2) wd
218   *  3) umask
219   */   
220  private final AbstractFileSystem defaultFS; //default FS for this FileContext.
221  private Path workingDir;          // Fully qualified
222  private FsPermission umask;
223  private final Configuration conf;
224  private final UserGroupInformation ugi;
225  final boolean resolveSymlinks;
226  private final Tracer tracer;
227
228  private FileContext(final AbstractFileSystem defFs,
229    final FsPermission theUmask, final Configuration aConf) {
230    defaultFS = defFs;
231    umask = FsPermission.getUMask(aConf);
232    conf = aConf;
233    tracer = FsTracer.get(aConf);
234    try {
235      ugi = UserGroupInformation.getCurrentUser();
236    } catch (IOException e) {
237      LOG.error("Exception in getCurrentUser: ",e);
238      throw new RuntimeException("Failed to get the current user " +
239                "while creating a FileContext", e);
240    }
241    /*
242     * Init the wd.
243     * WorkingDir is implemented at the FileContext layer 
244     * NOT at the AbstractFileSystem layer. 
245     * If the DefaultFS, such as localFilesystem has a notion of
246     *  builtin WD, we use that as the initial WD.
247     *  Otherwise the WD is initialized to the home directory.
248     */
249    workingDir = defaultFS.getInitialWorkingDirectory();
250    if (workingDir == null) {
251      workingDir = defaultFS.getHomeDirectory();
252    }
253    resolveSymlinks = conf.getBoolean(
254        CommonConfigurationKeys.FS_CLIENT_RESOLVE_REMOTE_SYMLINKS_KEY,
255        CommonConfigurationKeys.FS_CLIENT_RESOLVE_REMOTE_SYMLINKS_DEFAULT);
256    util = new Util(); // for the inner class
257  }
258
259  /* 
260   * Remove relative part - return "absolute":
261   * If input is relative path ("foo/bar") add wd: ie "/<workingDir>/foo/bar"
262   * A fully qualified uri ("hdfs://nn:p/foo/bar") or a slash-relative path
263   * ("/foo/bar") are returned unchanged.
264   * 
265   * Applications that use FileContext should use #makeQualified() since
266   * they really want a fully qualified URI.
267   * Hence this method is not called makeAbsolute() and 
268   * has been deliberately declared private.
269   */
270  Path fixRelativePart(Path p) {
271    Preconditions.checkNotNull(p, "path cannot be null");
272    if (p.isUriPathAbsolute()) {
273      return p;
274    } else {
275      return new Path(workingDir, p);
276    }
277  }
278
279  /**
280   * Delete all the paths that were marked as delete-on-exit.
281   */
282  static void processDeleteOnExit() {
283    synchronized (DELETE_ON_EXIT) {
284      Set<Entry<FileContext, Set<Path>>> set = DELETE_ON_EXIT.entrySet();
285      for (Entry<FileContext, Set<Path>> entry : set) {
286        FileContext fc = entry.getKey();
287        Set<Path> paths = entry.getValue();
288        for (Path path : paths) {
289          try {
290            fc.delete(path, true);
291          } catch (IOException e) {
292            LOG.warn("Ignoring failure to deleteOnExit for path " + path);
293          }
294        }
295      }
296      DELETE_ON_EXIT.clear();
297    }
298  }
299
300  /**
301   * Get the file system of supplied path.
302   * 
303   * @param absOrFqPath - absolute or fully qualified path
304   * @return the file system of the path
305   * 
306   * @throws UnsupportedFileSystemException If the file system for
307   *           <code>absOrFqPath</code> is not supported.
308   * @throws IOExcepton If the file system for <code>absOrFqPath</code> could
309   *         not be instantiated.
310   */
311  protected AbstractFileSystem getFSofPath(final Path absOrFqPath)
312      throws UnsupportedFileSystemException, IOException {
313    absOrFqPath.checkNotSchemeWithRelative();
314    absOrFqPath.checkNotRelative();
315
316    try { 
317      // Is it the default FS for this FileContext?
318      defaultFS.checkPath(absOrFqPath);
319      return defaultFS;
320    } catch (Exception e) { // it is different FileSystem
321      return getAbstractFileSystem(ugi, absOrFqPath.toUri(), conf);
322    }
323  }
324  
325  private static AbstractFileSystem getAbstractFileSystem(
326      UserGroupInformation user, final URI uri, final Configuration conf)
327      throws UnsupportedFileSystemException, IOException {
328    try {
329      return user.doAs(new PrivilegedExceptionAction<AbstractFileSystem>() {
330        @Override
331        public AbstractFileSystem run() throws UnsupportedFileSystemException {
332          return AbstractFileSystem.get(uri, conf);
333        }
334      });
335    } catch (InterruptedException ex) {
336      LOG.error(ex);
337      throw new IOException("Failed to get the AbstractFileSystem for path: "
338          + uri, ex);
339    }
340  }
341  
342  /**
343   * Protected Static Factory methods for getting a FileContexts
344   * that take a AbstractFileSystem as input. To be used for testing.
345   */
346
347  /**
348   * Create a FileContext with specified FS as default using the specified
349   * config.
350   * 
351   * @param defFS
352   * @param aConf
353   * @return new FileContext with specified FS as default.
354   */
355  public static FileContext getFileContext(final AbstractFileSystem defFS,
356                    final Configuration aConf) {
357    return new FileContext(defFS, FsPermission.getUMask(aConf), aConf);
358  }
359  
360  /**
361   * Create a FileContext for specified file system using the default config.
362   * 
363   * @param defaultFS
364   * @return a FileContext with the specified AbstractFileSystem
365   *                 as the default FS.
366   */
367  protected static FileContext getFileContext(
368    final AbstractFileSystem defaultFS) {
369    return getFileContext(defaultFS, new Configuration());
370  }
371 
372  /**
373   * Static Factory methods for getting a FileContext.
374   * Note new file contexts are created for each call.
375   * The only singleton is the local FS context using the default config.
376   * 
377   * Methods that use the default config: the default config read from the
378   * $HADOOP_CONFIG/core.xml,
379   * Unspecified key-values for config are defaulted from core-defaults.xml
380   * in the release jar.
381   * 
382   * The keys relevant to the FileContext layer are extracted at time of
383   * construction. Changes to the config after the call are ignore
384   * by the FileContext layer. 
385   * The conf is passed to lower layers like AbstractFileSystem and HDFS which
386   * pick up their own config variables.
387   */
388
389  /**
390   * Create a FileContext using the default config read from the
391   * $HADOOP_CONFIG/core.xml, Unspecified key-values for config are defaulted
392   * from core-defaults.xml in the release jar.
393   * 
394   * @throws UnsupportedFileSystemException If the file system from the default
395   *           configuration is not supported
396   */
397  public static FileContext getFileContext()
398      throws UnsupportedFileSystemException {
399    return getFileContext(new Configuration());
400  }
401
402  /**
403   * @return a FileContext for the local file system using the default config.
404   * @throws UnsupportedFileSystemException If the file system for
405   *           {@link FsConstants#LOCAL_FS_URI} is not supported.
406   */
407  public static FileContext getLocalFSFileContext()
408      throws UnsupportedFileSystemException {
409    return getFileContext(FsConstants.LOCAL_FS_URI);
410  }
411
412  /**
413   * Create a FileContext for specified URI using the default config.
414   * 
415   * @param defaultFsUri
416   * @return a FileContext with the specified URI as the default FS.
417   * 
418   * @throws UnsupportedFileSystemException If the file system for
419   *           <code>defaultFsUri</code> is not supported
420   */
421  public static FileContext getFileContext(final URI defaultFsUri)
422      throws UnsupportedFileSystemException {
423    return getFileContext(defaultFsUri, new Configuration());
424  }
425
426  /**
427   * Create a FileContext for specified default URI using the specified config.
428   * 
429   * @param defaultFsUri
430   * @param aConf
431   * @return new FileContext for specified uri
432   * @throws UnsupportedFileSystemException If the file system with specified is
433   *           not supported
434   * @throws RuntimeException If the file system specified is supported but
435   *         could not be instantiated, or if login fails.
436   */
437  public static FileContext getFileContext(final URI defaultFsUri,
438      final Configuration aConf) throws UnsupportedFileSystemException {
439    UserGroupInformation currentUser = null;
440    AbstractFileSystem defaultAfs = null;
441    if (defaultFsUri.getScheme() == null) {
442      return getFileContext(aConf);
443    }
444    try {
445      currentUser = UserGroupInformation.getCurrentUser();
446      defaultAfs = getAbstractFileSystem(currentUser, defaultFsUri, aConf);
447    } catch (UnsupportedFileSystemException ex) {
448      throw ex;
449    } catch (IOException ex) {
450      LOG.error(ex);
451      throw new RuntimeException(ex);
452    }
453    return getFileContext(defaultAfs, aConf);
454  }
455
456  /**
457   * Create a FileContext using the passed config. Generally it is better to use
458   * {@link #getFileContext(URI, Configuration)} instead of this one.
459   * 
460   * 
461   * @param aConf
462   * @return new FileContext
463   * @throws UnsupportedFileSystemException If file system in the config
464   *           is not supported
465   */
466  public static FileContext getFileContext(final Configuration aConf)
467      throws UnsupportedFileSystemException {
468    final URI defaultFsUri = URI.create(aConf.get(FS_DEFAULT_NAME_KEY,
469        FS_DEFAULT_NAME_DEFAULT));
470    if (   defaultFsUri.getScheme() != null
471        && !defaultFsUri.getScheme().trim().isEmpty()) {
472      return getFileContext(defaultFsUri, aConf);
473    }
474    throw new UnsupportedFileSystemException(String.format(
475        "%s: URI configured via %s carries no scheme",
476        defaultFsUri, FS_DEFAULT_NAME_KEY));
477  }
478
479  /**
480   * @param aConf - from which the FileContext is configured
481   * @return a FileContext for the local file system using the specified config.
482   * 
483   * @throws UnsupportedFileSystemException If default file system in the config
484   *           is not supported
485   * 
486   */
487  public static FileContext getLocalFSFileContext(final Configuration aConf)
488      throws UnsupportedFileSystemException {
489    return getFileContext(FsConstants.LOCAL_FS_URI, aConf);
490  }
491
492  /* This method is needed for tests. */
493  @InterfaceAudience.Private
494  @InterfaceStability.Unstable /* return type will change to AFS once
495                                  HADOOP-6223 is completed */
496  public AbstractFileSystem getDefaultFileSystem() {
497    return defaultFS;
498  }
499  
500  /**
501   * Set the working directory for wd-relative names (such a "foo/bar"). Working
502   * directory feature is provided by simply prefixing relative names with the
503   * working dir. Note this is different from Unix where the wd is actually set
504   * to the inode. Hence setWorkingDir does not follow symlinks etc. This works
505   * better in a distributed environment that has multiple independent roots.
506   * {@link #getWorkingDirectory()} should return what setWorkingDir() set.
507   * 
508   * @param newWDir new working directory
509   * @throws IOException 
510   * <br>
511   *           NewWdir can be one of:
512   *           <ul>
513   *           <li>relative path: "foo/bar";</li>
514   *           <li>absolute without scheme: "/foo/bar"</li>
515   *           <li>fully qualified with scheme: "xx://auth/foo/bar"</li>
516   *           </ul>
517   * <br>
518   *           Illegal WDs:
519   *           <ul>
520   *           <li>relative with scheme: "xx:foo/bar"</li>
521   *           <li>non existent directory</li>
522   *           </ul>
523   */
524  public void setWorkingDirectory(final Path newWDir) throws IOException {
525    newWDir.checkNotSchemeWithRelative();
526    /* wd is stored as a fully qualified path. We check if the given 
527     * path is not relative first since resolve requires and returns 
528     * an absolute path.
529     */  
530    final Path newWorkingDir = new Path(workingDir, newWDir);
531    FileStatus status = getFileStatus(newWorkingDir);
532    if (status.isFile()) {
533      throw new FileNotFoundException("Cannot setWD to a file");
534    }
535    workingDir = newWorkingDir;
536  }
537  
538  /**
539   * Gets the working directory for wd-relative names (such a "foo/bar").
540   */
541  public Path getWorkingDirectory() {
542    return workingDir;
543  }
544  
545  /**
546   * Gets the ugi in the file-context
547   * @return UserGroupInformation
548   */
549  public UserGroupInformation getUgi() {
550    return ugi;
551  }
552  
553  /**
554   * Return the current user's home directory in this file system.
555   * The default implementation returns "/user/$USER/".
556   * @return the home directory
557   */
558  public Path getHomeDirectory() {
559    return defaultFS.getHomeDirectory();
560  }
561  
562  /**
563   * 
564   * @return the umask of this FileContext
565   */
566  public FsPermission getUMask() {
567    return umask;
568  }
569  
570  /**
571   * Set umask to the supplied parameter.
572   * @param newUmask  the new umask
573   */
574  public void setUMask(final FsPermission newUmask) {
575    umask = newUmask;
576  }
577  
578  
579  /**
580   * Resolve the path following any symlinks or mount points
581   * @param f to be resolved
582   * @return fully qualified resolved path
583   * 
584   * @throws FileNotFoundException  If <code>f</code> does not exist
585   * @throws AccessControlException if access denied
586   * @throws IOException If an IO Error occurred
587   * 
588   * Exceptions applicable to file systems accessed over RPC:
589   * @throws RpcClientException If an exception occurred in the RPC client
590   * @throws RpcServerException If an exception occurred in the RPC server
591   * @throws UnexpectedServerException If server implementation throws
592   *           undeclared exception to RPC server
593   * 
594   * RuntimeExceptions:
595   * @throws InvalidPathException If path <code>f</code> is not valid
596   */
597  public Path resolvePath(final Path f) throws FileNotFoundException,
598      UnresolvedLinkException, AccessControlException, IOException {
599    return resolve(f);
600  }
601  
602  /**
603   * Make the path fully qualified if it is isn't. 
604   * A Fully-qualified path has scheme and authority specified and an absolute
605   * path.
606   * Use the default file system and working dir in this FileContext to qualify.
607   * @param path
608   * @return qualified path
609   */
610  public Path makeQualified(final Path path) {
611    return path.makeQualified(defaultFS.getUri(), getWorkingDirectory());
612  }
613
614  /**
615   * Create or overwrite file on indicated path and returns an output stream for
616   * writing into the file.
617   * 
618   * @param f the file name to open
619   * @param createFlag gives the semantics of create; see {@link CreateFlag}
620   * @param opts file creation options; see {@link Options.CreateOpts}.
621   *          <ul>
622   *          <li>Progress - to report progress on the operation - default null
623   *          <li>Permission - umask is applied against permission: default is
624   *          FsPermissions:getDefault()
625   * 
626   *          <li>CreateParent - create missing parent path; default is to not
627   *          to create parents
628   *          <li>The defaults for the following are SS defaults of the file
629   *          server implementing the target path. Not all parameters make sense
630   *          for all kinds of file system - eg. localFS ignores Blocksize,
631   *          replication, checksum
632   *          <ul>
633   *          <li>BufferSize - buffersize used in FSDataOutputStream
634   *          <li>Blocksize - block size for file blocks
635   *          <li>ReplicationFactor - replication for blocks
636   *          <li>ChecksumParam - Checksum parameters. server default is used
637   *          if not specified.
638   *          </ul>
639   *          </ul>
640   * 
641   * @return {@link FSDataOutputStream} for created file
642   * 
643   * @throws AccessControlException If access is denied
644   * @throws FileAlreadyExistsException If file <code>f</code> already exists
645   * @throws FileNotFoundException If parent of <code>f</code> does not exist
646   *           and <code>createParent</code> is false
647   * @throws ParentNotDirectoryException If parent of <code>f</code> is not a
648   *           directory.
649   * @throws UnsupportedFileSystemException If file system for <code>f</code> is
650   *           not supported
651   * @throws IOException If an I/O error occurred
652   * 
653   * Exceptions applicable to file systems accessed over RPC:
654   * @throws RpcClientException If an exception occurred in the RPC client
655   * @throws RpcServerException If an exception occurred in the RPC server
656   * @throws UnexpectedServerException If server implementation throws
657   *           undeclared exception to RPC server
658   * 
659   * RuntimeExceptions:
660   * @throws InvalidPathException If path <code>f</code> is not valid
661   */
662  public FSDataOutputStream create(final Path f,
663      final EnumSet<CreateFlag> createFlag, Options.CreateOpts... opts)
664      throws AccessControlException, FileAlreadyExistsException,
665      FileNotFoundException, ParentNotDirectoryException,
666      UnsupportedFileSystemException, IOException {
667    Path absF = fixRelativePart(f);
668
669    // If one of the options is a permission, extract it & apply umask
670    // If not, add a default Perms and apply umask;
671    // AbstractFileSystem#create
672
673    CreateOpts.Perms permOpt = CreateOpts.getOpt(CreateOpts.Perms.class, opts);
674    FsPermission permission = (permOpt != null) ? permOpt.getValue() :
675                                      FILE_DEFAULT_PERM;
676    permission = permission.applyUMask(umask);
677
678    final CreateOpts[] updatedOpts = 
679                      CreateOpts.setOpt(CreateOpts.perms(permission), opts);
680    return new FSLinkResolver<FSDataOutputStream>() {
681      @Override
682      public FSDataOutputStream next(final AbstractFileSystem fs, final Path p) 
683        throws IOException {
684        return fs.create(p, createFlag, updatedOpts);
685      }
686    }.resolve(this, absF);
687  }
688
689  /**
690   * Make(create) a directory and all the non-existent parents.
691   * 
692   * @param dir - the dir to make
693   * @param permission - permissions is set permission&~umask
694   * @param createParent - if true then missing parent dirs are created if false
695   *          then parent must exist
696   * 
697   * @throws AccessControlException If access is denied
698   * @throws FileAlreadyExistsException If directory <code>dir</code> already
699   *           exists
700   * @throws FileNotFoundException If parent of <code>dir</code> does not exist
701   *           and <code>createParent</code> is false
702   * @throws ParentNotDirectoryException If parent of <code>dir</code> is not a
703   *           directory
704   * @throws UnsupportedFileSystemException If file system for <code>dir</code>
705   *         is not supported
706   * @throws IOException If an I/O error occurred
707   * 
708   * Exceptions applicable to file systems accessed over RPC:
709   * @throws RpcClientException If an exception occurred in the RPC client
710   * @throws UnexpectedServerException If server implementation throws 
711   *           undeclared exception to RPC server
712   * 
713   * RuntimeExceptions:
714   * @throws InvalidPathException If path <code>dir</code> is not valid
715   */
716  public void mkdir(final Path dir, final FsPermission permission,
717      final boolean createParent) throws AccessControlException,
718      FileAlreadyExistsException, FileNotFoundException,
719      ParentNotDirectoryException, UnsupportedFileSystemException, 
720      IOException {
721    final Path absDir = fixRelativePart(dir);
722    final FsPermission absFerms = (permission == null ? 
723          FsPermission.getDirDefault() : permission).applyUMask(umask);
724    new FSLinkResolver<Void>() {
725      @Override
726      public Void next(final AbstractFileSystem fs, final Path p) 
727        throws IOException, UnresolvedLinkException {
728        fs.mkdir(p, absFerms, createParent);
729        return null;
730      }
731    }.resolve(this, absDir);
732  }
733
734  /**
735   * Delete a file.
736   * @param f the path to delete.
737   * @param recursive if path is a directory and set to 
738   * true, the directory is deleted else throws an exception. In
739   * case of a file the recursive can be set to either true or false.
740   *
741   * @throws AccessControlException If access is denied
742   * @throws FileNotFoundException If <code>f</code> does not exist
743   * @throws UnsupportedFileSystemException If file system for <code>f</code> is
744   *           not supported
745   * @throws IOException If an I/O error occurred
746   * 
747   * Exceptions applicable to file systems accessed over RPC:
748   * @throws RpcClientException If an exception occurred in the RPC client
749   * @throws RpcServerException If an exception occurred in the RPC server
750   * @throws UnexpectedServerException If server implementation throws 
751   *           undeclared exception to RPC server
752   * 
753   * RuntimeExceptions:
754   * @throws InvalidPathException If path <code>f</code> is invalid
755   */
756  public boolean delete(final Path f, final boolean recursive)
757      throws AccessControlException, FileNotFoundException,
758      UnsupportedFileSystemException, IOException {
759    Path absF = fixRelativePart(f);
760    return new FSLinkResolver<Boolean>() {
761      @Override
762      public Boolean next(final AbstractFileSystem fs, final Path p) 
763        throws IOException, UnresolvedLinkException {
764        return fs.delete(p, recursive);
765      }
766    }.resolve(this, absF);
767  }
768 
769  /**
770   * Opens an FSDataInputStream at the indicated Path using
771   * default buffersize.
772   * @param f the file name to open
773   *
774   * @throws AccessControlException If access is denied
775   * @throws FileNotFoundException If file <code>f</code> does not exist
776   * @throws UnsupportedFileSystemException If file system for <code>f</code>
777   *         is not supported
778   * @throws IOException If an I/O error occurred
779   * 
780   * Exceptions applicable to file systems accessed over RPC:
781   * @throws RpcClientException If an exception occurred in the RPC client
782   * @throws RpcServerException If an exception occurred in the RPC server
783   * @throws UnexpectedServerException If server implementation throws 
784   *           undeclared exception to RPC server
785   */
786  public FSDataInputStream open(final Path f) throws AccessControlException,
787      FileNotFoundException, UnsupportedFileSystemException, IOException {
788    final Path absF = fixRelativePart(f);
789    return new FSLinkResolver<FSDataInputStream>() {
790      @Override
791      public FSDataInputStream next(final AbstractFileSystem fs, final Path p) 
792        throws IOException, UnresolvedLinkException {
793        return fs.open(p);
794      }
795    }.resolve(this, absF);
796  }
797
798  /**
799   * Opens an FSDataInputStream at the indicated Path.
800   * 
801   * @param f the file name to open
802   * @param bufferSize the size of the buffer to be used.
803   * 
804   * @throws AccessControlException If access is denied
805   * @throws FileNotFoundException If file <code>f</code> does not exist
806   * @throws UnsupportedFileSystemException If file system for <code>f</code> is
807   *           not supported
808   * @throws IOException If an I/O error occurred
809   * 
810   * Exceptions applicable to file systems accessed over RPC:
811   * @throws RpcClientException If an exception occurred in the RPC client
812   * @throws RpcServerException If an exception occurred in the RPC server
813   * @throws UnexpectedServerException If server implementation throws 
814   *           undeclared exception to RPC server
815   */
816  public FSDataInputStream open(final Path f, final int bufferSize)
817      throws AccessControlException, FileNotFoundException,
818      UnsupportedFileSystemException, IOException {
819    final Path absF = fixRelativePart(f);
820    return new FSLinkResolver<FSDataInputStream>() {
821      @Override
822      public FSDataInputStream next(final AbstractFileSystem fs, final Path p) 
823        throws IOException, UnresolvedLinkException {
824        return fs.open(p, bufferSize);
825      }
826    }.resolve(this, absF);
827  }
828
829  /**
830   * Truncate the file in the indicated path to the indicated size.
831   * <ul>
832   * <li>Fails if path is a directory.
833   * <li>Fails if path does not exist.
834   * <li>Fails if path is not closed.
835   * <li>Fails if new size is greater than current size.
836   * </ul>
837   * @param f The path to the file to be truncated
838   * @param newLength The size the file is to be truncated to
839   *
840   * @return <code>true</code> if the file has been truncated to the desired
841   * <code>newLength</code> and is immediately available to be reused for
842   * write operations such as <code>append</code>, or
843   * <code>false</code> if a background process of adjusting the length of
844   * the last block has been started, and clients should wait for it to
845   * complete before proceeding with further file updates.
846   *
847   * @throws AccessControlException If access is denied
848   * @throws FileNotFoundException If file <code>f</code> does not exist
849   * @throws UnsupportedFileSystemException If file system for <code>f</code> is
850   *           not supported
851   * @throws IOException If an I/O error occurred
852   *
853   * Exceptions applicable to file systems accessed over RPC:
854   * @throws RpcClientException If an exception occurred in the RPC client
855   * @throws RpcServerException If an exception occurred in the RPC server
856   * @throws UnexpectedServerException If server implementation throws
857   *           undeclared exception to RPC server
858   */
859  public boolean truncate(final Path f, final long newLength)
860      throws AccessControlException, FileNotFoundException,
861      UnsupportedFileSystemException, IOException {
862    final Path absF = fixRelativePart(f);
863    return new FSLinkResolver<Boolean>() {
864      @Override
865      public Boolean next(final AbstractFileSystem fs, final Path p)
866          throws IOException, UnresolvedLinkException {
867        return fs.truncate(p, newLength);
868      }
869    }.resolve(this, absF);
870  }
871
872  /**
873   * Set replication for an existing file.
874   * 
875   * @param f file name
876   * @param replication new replication
877   *
878   * @return true if successful
879   *
880   * @throws AccessControlException If access is denied
881   * @throws FileNotFoundException If file <code>f</code> does not exist
882   * @throws IOException If an I/O error occurred
883   * 
884   * Exceptions applicable to file systems accessed over RPC:
885   * @throws RpcClientException If an exception occurred in the RPC client
886   * @throws RpcServerException If an exception occurred in the RPC server
887   * @throws UnexpectedServerException If server implementation throws 
888   *           undeclared exception to RPC server
889   */
890  public boolean setReplication(final Path f, final short replication)
891      throws AccessControlException, FileNotFoundException,
892      IOException {
893    final Path absF = fixRelativePart(f);
894    return new FSLinkResolver<Boolean>() {
895      @Override
896      public Boolean next(final AbstractFileSystem fs, final Path p) 
897        throws IOException, UnresolvedLinkException {
898        return fs.setReplication(p, replication);
899      }
900    }.resolve(this, absF);
901  }
902
903  /**
904   * Renames Path src to Path dst
905   * <ul>
906   * <li
907   * <li>Fails if src is a file and dst is a directory.
908   * <li>Fails if src is a directory and dst is a file.
909   * <li>Fails if the parent of dst does not exist or is a file.
910   * </ul>
911   * <p>
912   * If OVERWRITE option is not passed as an argument, rename fails if the dst
913   * already exists.
914   * <p>
915   * If OVERWRITE option is passed as an argument, rename overwrites the dst if
916   * it is a file or an empty directory. Rename fails if dst is a non-empty
917   * directory.
918   * <p>
919   * Note that atomicity of rename is dependent on the file system
920   * implementation. Please refer to the file system documentation for details
921   * <p>
922   * 
923   * @param src path to be renamed
924   * @param dst new path after rename
925   * 
926   * @throws AccessControlException If access is denied
927   * @throws FileAlreadyExistsException If <code>dst</code> already exists and
928   *           <code>options</options> has {@link Options.Rename#OVERWRITE} 
929   *           option false.
930   * @throws FileNotFoundException If <code>src</code> does not exist
931   * @throws ParentNotDirectoryException If parent of <code>dst</code> is not a
932   *           directory
933   * @throws UnsupportedFileSystemException If file system for <code>src</code>
934   *           and <code>dst</code> is not supported
935   * @throws IOException If an I/O error occurred
936   * 
937   * Exceptions applicable to file systems accessed over RPC:
938   * @throws RpcClientException If an exception occurred in the RPC client
939   * @throws RpcServerException If an exception occurred in the RPC server
940   * @throws UnexpectedServerException If server implementation throws
941   *           undeclared exception to RPC server
942   */
943  public void rename(final Path src, final Path dst,
944      final Options.Rename... options) throws AccessControlException,
945      FileAlreadyExistsException, FileNotFoundException,
946      ParentNotDirectoryException, UnsupportedFileSystemException,
947      IOException {
948    final Path absSrc = fixRelativePart(src);
949    final Path absDst = fixRelativePart(dst);
950    AbstractFileSystem srcFS = getFSofPath(absSrc);
951    AbstractFileSystem dstFS = getFSofPath(absDst);
952    if(!srcFS.getUri().equals(dstFS.getUri())) {
953      throw new IOException("Renames across AbstractFileSystems not supported");
954    }
955    try {
956      srcFS.rename(absSrc, absDst, options);
957    } catch (UnresolvedLinkException e) {
958      /* We do not know whether the source or the destination path
959       * was unresolved. Resolve the source path up until the final
960       * path component, then fully resolve the destination. 
961       */
962      final Path source = resolveIntermediate(absSrc);    
963      new FSLinkResolver<Void>() {
964        @Override
965        public Void next(final AbstractFileSystem fs, final Path p) 
966          throws IOException, UnresolvedLinkException {
967          fs.rename(source, p, options);
968          return null;
969        }
970      }.resolve(this, absDst);
971    }
972  }
973  
974  /**
975   * Set permission of a path.
976   * @param f
977   * @param permission - the new absolute permission (umask is not applied)
978   *
979   * @throws AccessControlException If access is denied
980   * @throws FileNotFoundException If <code>f</code> does not exist
981   * @throws UnsupportedFileSystemException If file system for <code>f</code>
982   *         is not supported
983   * @throws IOException If an I/O error occurred
984   * 
985   * Exceptions applicable to file systems accessed over RPC:
986   * @throws RpcClientException If an exception occurred in the RPC client
987   * @throws RpcServerException If an exception occurred in the RPC server
988   * @throws UnexpectedServerException If server implementation throws 
989   *           undeclared exception to RPC server
990   */
991  public void setPermission(final Path f, final FsPermission permission)
992      throws AccessControlException, FileNotFoundException,
993      UnsupportedFileSystemException, IOException {
994    final Path absF = fixRelativePart(f);
995    new FSLinkResolver<Void>() {
996      @Override
997      public Void next(final AbstractFileSystem fs, final Path p) 
998        throws IOException, UnresolvedLinkException {
999        fs.setPermission(p, permission);
1000        return null;
1001      }
1002    }.resolve(this, absF);
1003  }
1004
1005  /**
1006   * Set owner of a path (i.e. a file or a directory). The parameters username
1007   * and groupname cannot both be null.
1008   * 
1009   * @param f The path
1010   * @param username If it is null, the original username remains unchanged.
1011   * @param groupname If it is null, the original groupname remains unchanged.
1012   * 
1013   * @throws AccessControlException If access is denied
1014   * @throws FileNotFoundException If <code>f</code> does not exist
1015   * @throws UnsupportedFileSystemException If file system for <code>f</code> is
1016   *           not supported
1017   * @throws IOException If an I/O error occurred
1018   * 
1019   * Exceptions applicable to file systems accessed over RPC:
1020   * @throws RpcClientException If an exception occurred in the RPC client
1021   * @throws RpcServerException If an exception occurred in the RPC server
1022   * @throws UnexpectedServerException If server implementation throws 
1023   *           undeclared exception to RPC server
1024   * 
1025   * RuntimeExceptions:
1026   * @throws HadoopIllegalArgumentException If <code>username</code> or
1027   *           <code>groupname</code> is invalid.
1028   */
1029  public void setOwner(final Path f, final String username,
1030      final String groupname) throws AccessControlException,
1031      UnsupportedFileSystemException, FileNotFoundException,
1032      IOException {
1033    if ((username == null) && (groupname == null)) {
1034      throw new HadoopIllegalArgumentException(
1035          "username and groupname cannot both be null");
1036    }
1037    final Path absF = fixRelativePart(f);
1038    new FSLinkResolver<Void>() {
1039      @Override
1040      public Void next(final AbstractFileSystem fs, final Path p) 
1041        throws IOException, UnresolvedLinkException {
1042        fs.setOwner(p, username, groupname);
1043        return null;
1044      }
1045    }.resolve(this, absF);
1046  }
1047
1048  /**
1049   * Set access time of a file.
1050   * @param f The path
1051   * @param mtime Set the modification time of this file.
1052   *        The number of milliseconds since epoch (Jan 1, 1970). 
1053   *        A value of -1 means that this call should not set modification time.
1054   * @param atime Set the access time of this file.
1055   *        The number of milliseconds since Jan 1, 1970. 
1056   *        A value of -1 means that this call should not set access time.
1057   *
1058   * @throws AccessControlException If access is denied
1059   * @throws FileNotFoundException If <code>f</code> does not exist
1060   * @throws UnsupportedFileSystemException If file system for <code>f</code> is
1061   *           not supported
1062   * @throws IOException If an I/O error occurred
1063   * 
1064   * Exceptions applicable to file systems accessed over RPC:
1065   * @throws RpcClientException If an exception occurred in the RPC client
1066   * @throws RpcServerException If an exception occurred in the RPC server
1067   * @throws UnexpectedServerException If server implementation throws 
1068   *           undeclared exception to RPC server
1069   */
1070  public void setTimes(final Path f, final long mtime, final long atime)
1071      throws AccessControlException, FileNotFoundException,
1072      UnsupportedFileSystemException, IOException {
1073    final Path absF = fixRelativePart(f);
1074    new FSLinkResolver<Void>() {
1075      @Override
1076      public Void next(final AbstractFileSystem fs, final Path p) 
1077        throws IOException, UnresolvedLinkException {
1078        fs.setTimes(p, mtime, atime);
1079        return null;
1080      }
1081    }.resolve(this, absF);
1082  }
1083
1084  /**
1085   * Get the checksum of a file.
1086   *
1087   * @param f file path
1088   *
1089   * @return The file checksum.  The default return value is null,
1090   *  which indicates that no checksum algorithm is implemented
1091   *  in the corresponding FileSystem.
1092   *
1093   * @throws AccessControlException If access is denied
1094   * @throws FileNotFoundException If <code>f</code> does not exist
1095   * @throws IOException If an I/O error occurred
1096   * 
1097   * Exceptions applicable to file systems accessed over RPC:
1098   * @throws RpcClientException If an exception occurred in the RPC client
1099   * @throws RpcServerException If an exception occurred in the RPC server
1100   * @throws UnexpectedServerException If server implementation throws 
1101   *           undeclared exception to RPC server
1102   */
1103  public FileChecksum getFileChecksum(final Path f)
1104      throws AccessControlException, FileNotFoundException,
1105      IOException {
1106    final Path absF = fixRelativePart(f);
1107    return new FSLinkResolver<FileChecksum>() {
1108      @Override
1109      public FileChecksum next(final AbstractFileSystem fs, final Path p) 
1110        throws IOException, UnresolvedLinkException {
1111        return fs.getFileChecksum(p);
1112      }
1113    }.resolve(this, absF);
1114  }
1115
1116  /**
1117   * Set the verify checksum flag for the  file system denoted by the path.
1118   * This is only applicable if the 
1119   * corresponding FileSystem supports checksum. By default doesn't do anything.
1120   * @param verifyChecksum
1121   * @param f set the verifyChecksum for the Filesystem containing this path
1122   *
1123   * @throws AccessControlException If access is denied
1124   * @throws FileNotFoundException If <code>f</code> does not exist
1125   * @throws UnsupportedFileSystemException If file system for <code>f</code> is
1126   *           not supported
1127   * @throws IOException If an I/O error occurred
1128   * 
1129   * Exceptions applicable to file systems accessed over RPC:
1130   * @throws RpcClientException If an exception occurred in the RPC client
1131   * @throws RpcServerException If an exception occurred in the RPC server
1132   * @throws UnexpectedServerException If server implementation throws 
1133   *           undeclared exception to RPC server
1134   */
1135  public void setVerifyChecksum(final boolean verifyChecksum, final Path f)
1136      throws AccessControlException, FileNotFoundException,
1137      UnsupportedFileSystemException, IOException {
1138    final Path absF = resolve(fixRelativePart(f));
1139    getFSofPath(absF).setVerifyChecksum(verifyChecksum);
1140  }
1141
1142  /**
1143   * Return a file status object that represents the path.
1144   * @param f The path we want information from
1145   *
1146   * @return a FileStatus object
1147   *
1148   * @throws AccessControlException If access is denied
1149   * @throws FileNotFoundException If <code>f</code> does not exist
1150   * @throws UnsupportedFileSystemException If file system for <code>f</code> is
1151   *           not supported
1152   * @throws IOException If an I/O error occurred
1153   * 
1154   * Exceptions applicable to file systems accessed over RPC:
1155   * @throws RpcClientException If an exception occurred in the RPC client
1156   * @throws RpcServerException If an exception occurred in the RPC server
1157   * @throws UnexpectedServerException If server implementation throws 
1158   *           undeclared exception to RPC server
1159   */
1160  public FileStatus getFileStatus(final Path f) throws AccessControlException,
1161      FileNotFoundException, UnsupportedFileSystemException, IOException {
1162    final Path absF = fixRelativePart(f);
1163    return new FSLinkResolver<FileStatus>() {
1164      @Override
1165      public FileStatus next(final AbstractFileSystem fs, final Path p) 
1166        throws IOException, UnresolvedLinkException {
1167        return fs.getFileStatus(p);
1168      }
1169    }.resolve(this, absF);
1170  }
1171
1172  /**
1173   * Checks if the user can access a path.  The mode specifies which access
1174   * checks to perform.  If the requested permissions are granted, then the
1175   * method returns normally.  If access is denied, then the method throws an
1176   * {@link AccessControlException}.
1177   * <p/>
1178   * The default implementation of this method calls {@link #getFileStatus(Path)}
1179   * and checks the returned permissions against the requested permissions.
1180   * Note that the getFileStatus call will be subject to authorization checks.
1181   * Typically, this requires search (execute) permissions on each directory in
1182   * the path's prefix, but this is implementation-defined.  Any file system
1183   * that provides a richer authorization model (such as ACLs) may override the
1184   * default implementation so that it checks against that model instead.
1185   * <p>
1186   * In general, applications should avoid using this method, due to the risk of
1187   * time-of-check/time-of-use race conditions.  The permissions on a file may
1188   * change immediately after the access call returns.  Most applications should
1189   * prefer running specific file system actions as the desired user represented
1190   * by a {@link UserGroupInformation}.
1191   *
1192   * @param path Path to check
1193   * @param mode type of access to check
1194   * @throws AccessControlException if access is denied
1195   * @throws FileNotFoundException if the path does not exist
1196   * @throws UnsupportedFileSystemException if file system for <code>path</code>
1197   *   is not supported
1198   * @throws IOException see specific implementation
1199   * 
1200   * Exceptions applicable to file systems accessed over RPC:
1201   * @throws RpcClientException If an exception occurred in the RPC client
1202   * @throws RpcServerException If an exception occurred in the RPC server
1203   * @throws UnexpectedServerException If server implementation throws 
1204   *           undeclared exception to RPC server
1205   */
1206  @InterfaceAudience.LimitedPrivate({"HDFS", "Hive"})
1207  public void access(final Path path, final FsAction mode)
1208      throws AccessControlException, FileNotFoundException,
1209      UnsupportedFileSystemException, IOException {
1210    final Path absPath = fixRelativePart(path);
1211    new FSLinkResolver<Void>() {
1212      @Override
1213      public Void next(AbstractFileSystem fs, Path p) throws IOException,
1214          UnresolvedLinkException {
1215        fs.access(p, mode);
1216        return null;
1217      }
1218    }.resolve(this, absPath);
1219  }
1220
1221  /**
1222   * Return a file status object that represents the path. If the path 
1223   * refers to a symlink then the FileStatus of the symlink is returned.
1224   * The behavior is equivalent to #getFileStatus() if the underlying
1225   * file system does not support symbolic links.
1226   * @param  f The path we want information from.
1227   * @return A FileStatus object
1228   * 
1229   * @throws AccessControlException If access is denied
1230   * @throws FileNotFoundException If <code>f</code> does not exist
1231   * @throws UnsupportedFileSystemException If file system for <code>f</code> is
1232   *           not supported
1233   * @throws IOException If an I/O error occurred
1234   */
1235  public FileStatus getFileLinkStatus(final Path f)
1236      throws AccessControlException, FileNotFoundException,
1237      UnsupportedFileSystemException, IOException {
1238    final Path absF = fixRelativePart(f);
1239    return new FSLinkResolver<FileStatus>() {
1240      @Override
1241      public FileStatus next(final AbstractFileSystem fs, final Path p) 
1242        throws IOException, UnresolvedLinkException {
1243        FileStatus fi = fs.getFileLinkStatus(p);
1244        if (fi.isSymlink()) {
1245          fi.setSymlink(FSLinkResolver.qualifySymlinkTarget(fs.getUri(), p,
1246              fi.getSymlink()));
1247        }
1248        return fi;
1249      }
1250    }.resolve(this, absF);
1251  }
1252  
1253  /**
1254   * Returns the target of the given symbolic link as it was specified
1255   * when the link was created.  Links in the path leading up to the
1256   * final path component are resolved transparently.
1257   *
1258   * @param f the path to return the target of
1259   * @return The un-interpreted target of the symbolic link.
1260   * 
1261   * @throws AccessControlException If access is denied
1262   * @throws FileNotFoundException If path <code>f</code> does not exist
1263   * @throws UnsupportedFileSystemException If file system for <code>f</code> is
1264   *           not supported
1265   * @throws IOException If the given path does not refer to a symlink
1266   *           or an I/O error occurred
1267   */
1268  public Path getLinkTarget(final Path f) throws AccessControlException,
1269      FileNotFoundException, UnsupportedFileSystemException, IOException {
1270    final Path absF = fixRelativePart(f);
1271    return new FSLinkResolver<Path>() {
1272      @Override
1273      public Path next(final AbstractFileSystem fs, final Path p) 
1274        throws IOException, UnresolvedLinkException {
1275        FileStatus fi = fs.getFileLinkStatus(p);
1276        return fi.getSymlink();
1277      }
1278    }.resolve(this, absF);
1279  }
1280  
1281  /**
1282   * Return blockLocation of the given file for the given offset and len.
1283   *  For a nonexistent file or regions, null will be returned.
1284   *
1285   * This call is most helpful with DFS, where it returns 
1286   * hostnames of machines that contain the given file.
1287   * 
1288   * @param f - get blocklocations of this file
1289   * @param start position (byte offset)
1290   * @param len (in bytes)
1291   *
1292   * @return block locations for given file at specified offset of len
1293   *
1294   * @throws AccessControlException If access is denied
1295   * @throws FileNotFoundException If <code>f</code> does not exist
1296   * @throws UnsupportedFileSystemException If file system for <code>f</code> is
1297   *           not supported
1298   * @throws IOException If an I/O error occurred
1299   * 
1300   * Exceptions applicable to file systems accessed over RPC:
1301   * @throws RpcClientException If an exception occurred in the RPC client
1302   * @throws RpcServerException If an exception occurred in the RPC server
1303   * @throws UnexpectedServerException If server implementation throws 
1304   *           undeclared exception to RPC server
1305   * 
1306   * RuntimeExceptions:
1307   * @throws InvalidPathException If path <code>f</code> is invalid
1308   */
1309  @InterfaceAudience.LimitedPrivate({"HDFS", "MapReduce"})
1310  @InterfaceStability.Evolving
1311  public BlockLocation[] getFileBlockLocations(final Path f, final long start,
1312      final long len) throws AccessControlException, FileNotFoundException,
1313      UnsupportedFileSystemException, IOException {
1314    final Path absF = fixRelativePart(f);
1315    return new FSLinkResolver<BlockLocation[]>() {
1316      @Override
1317      public BlockLocation[] next(final AbstractFileSystem fs, final Path p) 
1318        throws IOException, UnresolvedLinkException {
1319        return fs.getFileBlockLocations(p, start, len);
1320      }
1321    }.resolve(this, absF);
1322  }
1323  
1324  /**
1325   * Returns a status object describing the use and capacity of the
1326   * file system denoted by the Parh argument p.
1327   * If the file system has multiple partitions, the
1328   * use and capacity of the partition pointed to by the specified
1329   * path is reflected.
1330   * 
1331   * @param f Path for which status should be obtained. null means the
1332   * root partition of the default file system. 
1333   *
1334   * @return a FsStatus object
1335   *
1336   * @throws AccessControlException If access is denied
1337   * @throws FileNotFoundException If <code>f</code> does not exist
1338   * @throws UnsupportedFileSystemException If file system for <code>f</code> is
1339   *           not supported
1340   * @throws IOException If an I/O error occurred
1341   * 
1342   * Exceptions applicable to file systems accessed over RPC:
1343   * @throws RpcClientException If an exception occurred in the RPC client
1344   * @throws RpcServerException If an exception occurred in the RPC server
1345   * @throws UnexpectedServerException If server implementation throws 
1346   *           undeclared exception to RPC server
1347   */
1348  public FsStatus getFsStatus(final Path f) throws AccessControlException,
1349      FileNotFoundException, UnsupportedFileSystemException, IOException {
1350    if (f == null) {
1351      return defaultFS.getFsStatus();
1352    }
1353    final Path absF = fixRelativePart(f);
1354    return new FSLinkResolver<FsStatus>() {
1355      @Override
1356      public FsStatus next(final AbstractFileSystem fs, final Path p) 
1357        throws IOException, UnresolvedLinkException {
1358        return fs.getFsStatus(p);
1359      }
1360    }.resolve(this, absF);
1361  }
1362
1363  /**
1364   * Creates a symbolic link to an existing file. An exception is thrown if 
1365   * the symlink exits, the user does not have permission to create symlink,
1366   * or the underlying file system does not support symlinks.
1367   * 
1368   * Symlink permissions are ignored, access to a symlink is determined by
1369   * the permissions of the symlink target.
1370   * 
1371   * Symlinks in paths leading up to the final path component are resolved 
1372   * transparently. If the final path component refers to a symlink some 
1373   * functions operate on the symlink itself, these are:
1374   * - delete(f) and deleteOnExit(f) - Deletes the symlink.
1375   * - rename(src, dst) - If src refers to a symlink, the symlink is 
1376   *   renamed. If dst refers to a symlink, the symlink is over-written.
1377   * - getLinkTarget(f) - Returns the target of the symlink. 
1378   * - getFileLinkStatus(f) - Returns a FileStatus object describing
1379   *   the symlink.
1380   * Some functions, create() and mkdir(), expect the final path component
1381   * does not exist. If they are given a path that refers to a symlink that 
1382   * does exist they behave as if the path referred to an existing file or 
1383   * directory. All other functions fully resolve, ie follow, the symlink. 
1384   * These are: open, setReplication, setOwner, setTimes, setWorkingDirectory,
1385   * setPermission, getFileChecksum, setVerifyChecksum, getFileBlockLocations,
1386   * getFsStatus, getFileStatus, exists, and listStatus.
1387   * 
1388   * Symlink targets are stored as given to createSymlink, assuming the 
1389   * underlying file system is capable of storing a fully qualified URI.
1390   * Dangling symlinks are permitted. FileContext supports four types of 
1391   * symlink targets, and resolves them as follows
1392   * <pre>
1393   * Given a path referring to a symlink of form:
1394   * 
1395   *   <---X---> 
1396   *   fs://host/A/B/link 
1397   *   <-----Y----->
1398   * 
1399   * In this path X is the scheme and authority that identify the file system,
1400   * and Y is the path leading up to the final path component "link". If Y is
1401   * a symlink  itself then let Y' be the target of Y and X' be the scheme and
1402   * authority of Y'. Symlink targets may:
1403   * 
1404   * 1. Fully qualified URIs
1405   * 
1406   * fs://hostX/A/B/file  Resolved according to the target file system.
1407   * 
1408   * 2. Partially qualified URIs (eg scheme but no host)
1409   * 
1410   * fs:///A/B/file  Resolved according to the target file system. Eg resolving
1411   *                 a symlink to hdfs:///A results in an exception because
1412   *                 HDFS URIs must be fully qualified, while a symlink to 
1413   *                 file:///A will not since Hadoop's local file systems 
1414   *                 require partially qualified URIs.
1415   * 
1416   * 3. Relative paths
1417   * 
1418   * path  Resolves to [Y'][path]. Eg if Y resolves to hdfs://host/A and path 
1419   *       is "../B/file" then [Y'][path] is hdfs://host/B/file
1420   * 
1421   * 4. Absolute paths
1422   * 
1423   * path  Resolves to [X'][path]. Eg if Y resolves hdfs://host/A/B and path
1424   *       is "/file" then [X][path] is hdfs://host/file
1425   * </pre>
1426   * 
1427   * @param target the target of the symbolic link
1428   * @param link the path to be created that points to target
1429   * @param createParent if true then missing parent dirs are created if 
1430   *                     false then parent must exist
1431   *
1432   *
1433   * @throws AccessControlException If access is denied
1434   * @throws FileAlreadyExistsException If file <code>linkcode> already exists
1435   * @throws FileNotFoundException If <code>target</code> does not exist
1436   * @throws ParentNotDirectoryException If parent of <code>link</code> is not a
1437   *           directory.
1438   * @throws UnsupportedFileSystemException If file system for 
1439   *           <code>target</code> or <code>link</code> is not supported
1440   * @throws IOException If an I/O error occurred
1441   */
1442  @SuppressWarnings("deprecation")
1443  public void createSymlink(final Path target, final Path link,
1444      final boolean createParent) throws AccessControlException,
1445      FileAlreadyExistsException, FileNotFoundException,
1446      ParentNotDirectoryException, UnsupportedFileSystemException, 
1447      IOException { 
1448    if (!FileSystem.areSymlinksEnabled()) {
1449      throw new UnsupportedOperationException("Symlinks not supported");
1450    }
1451    final Path nonRelLink = fixRelativePart(link);
1452    new FSLinkResolver<Void>() {
1453      @Override
1454      public Void next(final AbstractFileSystem fs, final Path p) 
1455        throws IOException, UnresolvedLinkException {
1456        fs.createSymlink(target, p, createParent);
1457        return null;
1458      }
1459    }.resolve(this, nonRelLink);
1460  }
1461  
1462  /**
1463   * List the statuses of the files/directories in the given path if the path is
1464   * a directory.
1465   * 
1466   * @param f is the path
1467   *
1468   * @return an iterator that traverses statuses of the files/directories 
1469   *         in the given path
1470   *
1471   * @throws AccessControlException If access is denied
1472   * @throws FileNotFoundException If <code>f</code> does not exist
1473   * @throws UnsupportedFileSystemException If file system for <code>f</code> is
1474   *           not supported
1475   * @throws IOException If an I/O error occurred
1476   * 
1477   * Exceptions applicable to file systems accessed over RPC:
1478   * @throws RpcClientException If an exception occurred in the RPC client
1479   * @throws RpcServerException If an exception occurred in the RPC server
1480   * @throws UnexpectedServerException If server implementation throws 
1481   *           undeclared exception to RPC server
1482   */
1483  public RemoteIterator<FileStatus> listStatus(final Path f) throws
1484      AccessControlException, FileNotFoundException,
1485      UnsupportedFileSystemException, IOException {
1486    final Path absF = fixRelativePart(f);
1487    return new FSLinkResolver<RemoteIterator<FileStatus>>() {
1488      @Override
1489      public RemoteIterator<FileStatus> next(
1490          final AbstractFileSystem fs, final Path p) 
1491        throws IOException, UnresolvedLinkException {
1492        return fs.listStatusIterator(p);
1493      }
1494    }.resolve(this, absF);
1495  }
1496
1497  /**
1498   * @return an iterator over the corrupt files under the given path
1499   * (may contain duplicates if a file has more than one corrupt block)
1500   * @throws IOException
1501   */
1502  public RemoteIterator<Path> listCorruptFileBlocks(Path path)
1503    throws IOException {
1504    final Path absF = fixRelativePart(path);
1505    return new FSLinkResolver<RemoteIterator<Path>>() {
1506      @Override
1507      public RemoteIterator<Path> next(final AbstractFileSystem fs,
1508                                       final Path p) 
1509        throws IOException, UnresolvedLinkException {
1510        return fs.listCorruptFileBlocks(p);
1511      }
1512    }.resolve(this, absF);
1513  }
1514  
1515  /**
1516   * List the statuses of the files/directories in the given path if the path is
1517   * a directory. 
1518   * Return the file's status and block locations If the path is a file.
1519   * 
1520   * If a returned status is a file, it contains the file's block locations.
1521   * 
1522   * @param f is the path
1523   *
1524   * @return an iterator that traverses statuses of the files/directories 
1525   *         in the given path
1526   * If any IO exception (for example the input directory gets deleted while
1527   * listing is being executed), next() or hasNext() of the returned iterator
1528   * may throw a RuntimeException with the io exception as the cause.
1529   *
1530   * @throws AccessControlException If access is denied
1531   * @throws FileNotFoundException If <code>f</code> does not exist
1532   * @throws UnsupportedFileSystemException If file system for <code>f</code> is
1533   *           not supported
1534   * @throws IOException If an I/O error occurred
1535   * 
1536   * Exceptions applicable to file systems accessed over RPC:
1537   * @throws RpcClientException If an exception occurred in the RPC client
1538   * @throws RpcServerException If an exception occurred in the RPC server
1539   * @throws UnexpectedServerException If server implementation throws 
1540   *           undeclared exception to RPC server
1541   */
1542  public RemoteIterator<LocatedFileStatus> listLocatedStatus(
1543      final Path f) throws
1544      AccessControlException, FileNotFoundException,
1545      UnsupportedFileSystemException, IOException {
1546    final Path absF = fixRelativePart(f);
1547    return new FSLinkResolver<RemoteIterator<LocatedFileStatus>>() {
1548      @Override
1549      public RemoteIterator<LocatedFileStatus> next(
1550          final AbstractFileSystem fs, final Path p) 
1551        throws IOException, UnresolvedLinkException {
1552        return fs.listLocatedStatus(p);
1553      }
1554    }.resolve(this, absF);
1555  }
1556
1557  /**
1558   * Mark a path to be deleted on JVM shutdown.
1559   * 
1560   * @param f the existing path to delete.
1561   *
1562   * @return  true if deleteOnExit is successful, otherwise false.
1563   *
1564   * @throws AccessControlException If access is denied
1565   * @throws UnsupportedFileSystemException If file system for <code>f</code> is
1566   *           not supported
1567   * @throws IOException If an I/O error occurred
1568   * 
1569   * Exceptions applicable to file systems accessed over RPC:
1570   * @throws RpcClientException If an exception occurred in the RPC client
1571   * @throws RpcServerException If an exception occurred in the RPC server
1572   * @throws UnexpectedServerException If server implementation throws 
1573   *           undeclared exception to RPC server
1574   */
1575  public boolean deleteOnExit(Path f) throws AccessControlException,
1576      IOException {
1577    if (!this.util().exists(f)) {
1578      return false;
1579    }
1580    synchronized (DELETE_ON_EXIT) {
1581      if (DELETE_ON_EXIT.isEmpty()) {
1582        ShutdownHookManager.get().addShutdownHook(FINALIZER, SHUTDOWN_HOOK_PRIORITY);
1583      }
1584      
1585      Set<Path> set = DELETE_ON_EXIT.get(this);
1586      if (set == null) {
1587        set = new TreeSet<Path>();
1588        DELETE_ON_EXIT.put(this, set);
1589      }
1590      set.add(f);
1591    }
1592    return true;
1593  }
1594  
1595  private final Util util;
1596  public Util util() {
1597    return util;
1598  }
1599  
1600  
1601  /**
1602   * Utility/library methods built over the basic FileContext methods.
1603   * Since this are library functions, the oprtation are not atomic
1604   * and some of them may partially complete if other threads are making
1605   * changes to the same part of the name space.
1606   */
1607  public class Util {
1608    /**
1609     * Does the file exist?
1610     * Note: Avoid using this method if you already have FileStatus in hand.
1611     * Instead reuse the FileStatus 
1612     * @param f the  file or dir to be checked
1613     *
1614     * @throws AccessControlException If access is denied
1615     * @throws IOException If an I/O error occurred
1616     * @throws UnsupportedFileSystemException If file system for <code>f</code> is
1617     *           not supported
1618     * 
1619     * Exceptions applicable to file systems accessed over RPC:
1620     * @throws RpcClientException If an exception occurred in the RPC client
1621     * @throws RpcServerException If an exception occurred in the RPC server
1622     * @throws UnexpectedServerException If server implementation throws 
1623     *           undeclared exception to RPC server
1624     */
1625    public boolean exists(final Path f) throws AccessControlException,
1626      UnsupportedFileSystemException, IOException {
1627      try {
1628        FileStatus fs = FileContext.this.getFileStatus(f);
1629        assert fs != null;
1630        return true;
1631      } catch (FileNotFoundException e) {
1632        return false;
1633      }
1634    }
1635    
1636    /**
1637     * Return the {@link ContentSummary} of path f.
1638     * @param f path
1639     *
1640     * @return the {@link ContentSummary} of path f.
1641     *
1642     * @throws AccessControlException If access is denied
1643     * @throws FileNotFoundException If <code>f</code> does not exist
1644     * @throws UnsupportedFileSystemException If file system for 
1645     *         <code>f</code> is not supported
1646     * @throws IOException If an I/O error occurred
1647     * 
1648     * Exceptions applicable to file systems accessed over RPC:
1649     * @throws RpcClientException If an exception occurred in the RPC client
1650     * @throws RpcServerException If an exception occurred in the RPC server
1651     * @throws UnexpectedServerException If server implementation throws 
1652     *           undeclared exception to RPC server
1653     */
1654    public ContentSummary getContentSummary(Path f)
1655        throws AccessControlException, FileNotFoundException,
1656        UnsupportedFileSystemException, IOException {
1657      FileStatus status = FileContext.this.getFileStatus(f);
1658      if (status.isFile()) {
1659        long length = status.getLen();
1660        return new ContentSummary.Builder().length(length).
1661            fileCount(1).directoryCount(0).spaceConsumed(length).
1662            build();
1663      }
1664      long[] summary = {0, 0, 1};
1665      RemoteIterator<FileStatus> statusIterator =
1666        FileContext.this.listStatus(f);
1667      while(statusIterator.hasNext()) {
1668        FileStatus s = statusIterator.next();
1669        long length = s.getLen();
1670        ContentSummary c = s.isDirectory() ? getContentSummary(s.getPath()) :
1671            new ContentSummary.Builder().length(length).fileCount(1).
1672            directoryCount(0).spaceConsumed(length).build();
1673        summary[0] += c.getLength();
1674        summary[1] += c.getFileCount();
1675        summary[2] += c.getDirectoryCount();
1676      }
1677      return new ContentSummary.Builder().length(summary[0]).
1678          fileCount(summary[1]).directoryCount(summary[2]).
1679          spaceConsumed(summary[0]).build();
1680    }
1681    
1682    /**
1683     * See {@link #listStatus(Path[], PathFilter)}
1684     */
1685    public FileStatus[] listStatus(Path[] files) throws AccessControlException,
1686        FileNotFoundException, IOException {
1687      return listStatus(files, DEFAULT_FILTER);
1688    }
1689     
1690    /**
1691     * Filter files/directories in the given path using the user-supplied path
1692     * filter.
1693     * 
1694     * @param f is the path name
1695     * @param filter is the user-supplied path filter
1696     *
1697     * @return an array of FileStatus objects for the files under the given path
1698     *         after applying the filter
1699     *
1700     * @throws AccessControlException If access is denied
1701     * @throws FileNotFoundException If <code>f</code> does not exist
1702     * @throws UnsupportedFileSystemException If file system for 
1703     *         <code>pathPattern</code> is not supported
1704     * @throws IOException If an I/O error occurred
1705     * 
1706     * Exceptions applicable to file systems accessed over RPC:
1707     * @throws RpcClientException If an exception occurred in the RPC client
1708     * @throws RpcServerException If an exception occurred in the RPC server
1709     * @throws UnexpectedServerException If server implementation throws 
1710     *           undeclared exception to RPC server
1711     */
1712    public FileStatus[] listStatus(Path f, PathFilter filter)
1713        throws AccessControlException, FileNotFoundException,
1714        UnsupportedFileSystemException, IOException {
1715      ArrayList<FileStatus> results = new ArrayList<FileStatus>();
1716      listStatus(results, f, filter);
1717      return results.toArray(new FileStatus[results.size()]);
1718    }
1719    
1720    /**
1721     * Filter files/directories in the given list of paths using user-supplied
1722     * path filter.
1723     * 
1724     * @param files is a list of paths
1725     * @param filter is the filter
1726     *
1727     * @return a list of statuses for the files under the given paths after
1728     *         applying the filter
1729     *
1730     * @throws AccessControlException If access is denied
1731     * @throws FileNotFoundException If a file in <code>files</code> does not 
1732     *           exist
1733     * @throws IOException If an I/O error occurred
1734     * 
1735     * Exceptions applicable to file systems accessed over RPC:
1736     * @throws RpcClientException If an exception occurred in the RPC client
1737     * @throws RpcServerException If an exception occurred in the RPC server
1738     * @throws UnexpectedServerException If server implementation throws 
1739     *           undeclared exception to RPC server
1740     */
1741    public FileStatus[] listStatus(Path[] files, PathFilter filter)
1742        throws AccessControlException, FileNotFoundException, IOException {
1743      ArrayList<FileStatus> results = new ArrayList<FileStatus>();
1744      for (int i = 0; i < files.length; i++) {
1745        listStatus(results, files[i], filter);
1746      }
1747      return results.toArray(new FileStatus[results.size()]);
1748    }
1749  
1750    /*
1751     * Filter files/directories in the given path using the user-supplied path
1752     * filter. Results are added to the given array <code>results</code>.
1753     */
1754    private void listStatus(ArrayList<FileStatus> results, Path f,
1755        PathFilter filter) throws AccessControlException,
1756        FileNotFoundException, IOException {
1757      FileStatus[] listing = listStatus(f);
1758      if (listing != null) {
1759        for (int i = 0; i < listing.length; i++) {
1760          if (filter.accept(listing[i].getPath())) {
1761            results.add(listing[i]);
1762          }
1763        }
1764      }
1765    }
1766
1767    /**
1768     * List the statuses of the files/directories in the given path 
1769     * if the path is a directory.
1770     * 
1771     * @param f is the path
1772     *
1773     * @return an array that contains statuses of the files/directories 
1774     *         in the given path
1775     *
1776     * @throws AccessControlException If access is denied
1777     * @throws FileNotFoundException If <code>f</code> does not exist
1778     * @throws UnsupportedFileSystemException If file system for <code>f</code> is
1779     *           not supported
1780     * @throws IOException If an I/O error occurred
1781     * 
1782     * Exceptions applicable to file systems accessed over RPC:
1783     * @throws RpcClientException If an exception occurred in the RPC client
1784     * @throws RpcServerException If an exception occurred in the RPC server
1785     * @throws UnexpectedServerException If server implementation throws 
1786     *           undeclared exception to RPC server
1787     */
1788    public FileStatus[] listStatus(final Path f) throws AccessControlException,
1789        FileNotFoundException, UnsupportedFileSystemException,
1790        IOException {
1791      final Path absF = fixRelativePart(f);
1792      return new FSLinkResolver<FileStatus[]>() {
1793        @Override
1794        public FileStatus[] next(final AbstractFileSystem fs, final Path p) 
1795          throws IOException, UnresolvedLinkException {
1796          return fs.listStatus(p);
1797        }
1798      }.resolve(FileContext.this, absF);
1799    }
1800
1801    /**
1802     * List the statuses and block locations of the files in the given path.
1803     * 
1804     * If the path is a directory, 
1805     *   if recursive is false, returns files in the directory;
1806     *   if recursive is true, return files in the subtree rooted at the path.
1807     *   The subtree is traversed in the depth-first order.
1808     * If the path is a file, return the file's status and block locations.
1809     * Files across symbolic links are also returned.
1810     * 
1811     * @param f is the path
1812     * @param recursive if the subdirectories need to be traversed recursively
1813     *
1814     * @return an iterator that traverses statuses of the files
1815     * If any IO exception (for example a sub-directory gets deleted while
1816     * listing is being executed), next() or hasNext() of the returned iterator
1817     * may throw a RuntimeException with the IO exception as the cause.
1818     *
1819     * @throws AccessControlException If access is denied
1820     * @throws FileNotFoundException If <code>f</code> does not exist
1821     * @throws UnsupportedFileSystemException If file system for <code>f</code>
1822     *         is not supported
1823     * @throws IOException If an I/O error occurred
1824     * 
1825     * Exceptions applicable to file systems accessed over RPC:
1826     * @throws RpcClientException If an exception occurred in the RPC client
1827     * @throws RpcServerException If an exception occurred in the RPC server
1828     * @throws UnexpectedServerException If server implementation throws 
1829     *           undeclared exception to RPC server
1830     */
1831    public RemoteIterator<LocatedFileStatus> listFiles(
1832        final Path f, final boolean recursive) throws AccessControlException,
1833        FileNotFoundException, UnsupportedFileSystemException, 
1834        IOException {
1835      return new RemoteIterator<LocatedFileStatus>() {
1836        private Stack<RemoteIterator<LocatedFileStatus>> itors = 
1837          new Stack<RemoteIterator<LocatedFileStatus>>();
1838        RemoteIterator<LocatedFileStatus> curItor = listLocatedStatus(f);
1839        LocatedFileStatus curFile;
1840
1841        /**
1842         * Returns <tt>true</tt> if the iterator has more files.
1843         *
1844         * @return <tt>true</tt> if the iterator has more files.
1845         * @throws AccessControlException if not allowed to access next
1846         *                                file's status or locations
1847         * @throws FileNotFoundException if next file does not exist any more
1848         * @throws UnsupportedFileSystemException if next file's 
1849         *                                        fs is unsupported
1850         * @throws IOException for all other IO errors
1851         *                     for example, NameNode is not avaialbe or
1852         *                     NameNode throws IOException due to an error
1853         *                     while getting the status or block locations
1854         */
1855        @Override
1856        public boolean hasNext() throws IOException {
1857          while (curFile == null) {
1858            if (curItor.hasNext()) {
1859              handleFileStat(curItor.next());
1860            } else if (!itors.empty()) {
1861              curItor = itors.pop();
1862            } else {
1863              return false;
1864            }
1865          }
1866          return true;
1867        }
1868
1869        /**
1870         * Process the input stat.
1871         * If it is a file, return the file stat.
1872         * If it is a directory, traverse the directory if recursive is true;
1873         * ignore it if recursive is false.
1874         * If it is a symlink, resolve the symlink first and then process it
1875         * depending on if it is a file or directory.
1876         * @param stat input status
1877         * @throws AccessControlException if access is denied
1878         * @throws FileNotFoundException if file is not found
1879         * @throws UnsupportedFileSystemException if fs is not supported
1880         * @throws IOException for all other IO errors
1881         */
1882        private void handleFileStat(LocatedFileStatus stat)
1883        throws IOException {
1884          if (stat.isFile()) { // file
1885            curFile = stat;
1886          } else if (stat.isSymlink()) { // symbolic link
1887            // resolve symbolic link
1888            FileStatus symstat = FileContext.this.getFileStatus(
1889                stat.getSymlink());
1890            if (symstat.isFile() || (recursive && symstat.isDirectory())) {
1891              itors.push(curItor);
1892              curItor = listLocatedStatus(stat.getPath());
1893            }
1894          } else if (recursive) { // directory
1895            itors.push(curItor);
1896            curItor = listLocatedStatus(stat.getPath());
1897          }
1898        }
1899
1900        /**
1901         * Returns the next file's status with its block locations
1902         *
1903         * @throws AccessControlException if not allowed to access next
1904         *                                file's status or locations
1905         * @throws FileNotFoundException if next file does not exist any more
1906         * @throws UnsupportedFileSystemException if next file's 
1907         *                                        fs is unsupported
1908         * @throws IOException for all other IO errors
1909         *                     for example, NameNode is not avaialbe or
1910         *                     NameNode throws IOException due to an error
1911         *                     while getting the status or block locations
1912         */
1913        @Override
1914        public LocatedFileStatus next() throws IOException {
1915          if (hasNext()) {
1916            LocatedFileStatus result = curFile;
1917            curFile = null;
1918            return result;
1919          } 
1920          throw new java.util.NoSuchElementException("No more entry in " + f);
1921        }
1922      };
1923    }
1924
1925    /**
1926     * <p>Return all the files that match filePattern and are not checksum
1927     * files. Results are sorted by their names.
1928     * 
1929     * <p>
1930     * A filename pattern is composed of <i>regular</i> characters and
1931     * <i>special pattern matching</i> characters, which are:
1932     *
1933     * <dl>
1934     *  <dd>
1935     *   <dl>
1936     *    <p>
1937     *    <dt> <tt> ? </tt>
1938     *    <dd> Matches any single character.
1939     *
1940     *    <p>
1941     *    <dt> <tt> * </tt>
1942     *    <dd> Matches zero or more characters.
1943     *
1944     *    <p>
1945     *    <dt> <tt> [<i>abc</i>] </tt>
1946     *    <dd> Matches a single character from character set
1947     *     <tt>{<i>a,b,c</i>}</tt>.
1948     *
1949     *    <p>
1950     *    <dt> <tt> [<i>a</i>-<i>b</i>] </tt>
1951     *    <dd> Matches a single character from the character range
1952     *     <tt>{<i>a...b</i>}</tt>. Note: character <tt><i>a</i></tt> must be
1953     *     lexicographically less than or equal to character <tt><i>b</i></tt>.
1954     *
1955     *    <p>
1956     *    <dt> <tt> [^<i>a</i>] </tt>
1957     *    <dd> Matches a single char that is not from character set or range
1958     *     <tt>{<i>a</i>}</tt>.  Note that the <tt>^</tt> character must occur
1959     *     immediately to the right of the opening bracket.
1960     *
1961     *    <p>
1962     *    <dt> <tt> \<i>c</i> </tt>
1963     *    <dd> Removes (escapes) any special meaning of character <i>c</i>.
1964     *
1965     *    <p>
1966     *    <dt> <tt> {ab,cd} </tt>
1967     *    <dd> Matches a string from the string set <tt>{<i>ab, cd</i>} </tt>
1968     *    
1969     *    <p>
1970     *    <dt> <tt> {ab,c{de,fh}} </tt>
1971     *    <dd> Matches a string from string set <tt>{<i>ab, cde, cfh</i>}</tt>
1972     *
1973     *   </dl>
1974     *  </dd>
1975     * </dl>
1976     *
1977     * @param pathPattern a regular expression specifying a pth pattern
1978     *
1979     * @return an array of paths that match the path pattern
1980     *
1981     * @throws AccessControlException If access is denied
1982     * @throws UnsupportedFileSystemException If file system for 
1983     *         <code>pathPattern</code> is not supported
1984     * @throws IOException If an I/O error occurred
1985     * 
1986     * Exceptions applicable to file systems accessed over RPC:
1987     * @throws RpcClientException If an exception occurred in the RPC client
1988     * @throws RpcServerException If an exception occurred in the RPC server
1989     * @throws UnexpectedServerException If server implementation throws 
1990     *           undeclared exception to RPC server
1991     */
1992    public FileStatus[] globStatus(Path pathPattern)
1993        throws AccessControlException, UnsupportedFileSystemException,
1994        IOException {
1995      return new Globber(FileContext.this, pathPattern, DEFAULT_FILTER).glob();
1996    }
1997    
1998    /**
1999     * Return an array of FileStatus objects whose path names match pathPattern
2000     * and is accepted by the user-supplied path filter. Results are sorted by
2001     * their path names.
2002     * Return null if pathPattern has no glob and the path does not exist.
2003     * Return an empty array if pathPattern has a glob and no path matches it. 
2004     * 
2005     * @param pathPattern regular expression specifying the path pattern
2006     * @param filter user-supplied path filter
2007     *
2008     * @return an array of FileStatus objects
2009     *
2010     * @throws AccessControlException If access is denied
2011     * @throws UnsupportedFileSystemException If file system for 
2012     *         <code>pathPattern</code> is not supported
2013     * @throws IOException If an I/O error occurred
2014     * 
2015     * Exceptions applicable to file systems accessed over RPC:
2016     * @throws RpcClientException If an exception occurred in the RPC client
2017     * @throws RpcServerException If an exception occurred in the RPC server
2018     * @throws UnexpectedServerException If server implementation throws 
2019     *           undeclared exception to RPC server
2020     */
2021    public FileStatus[] globStatus(final Path pathPattern,
2022        final PathFilter filter) throws AccessControlException,
2023        UnsupportedFileSystemException, IOException {
2024      return new Globber(FileContext.this, pathPattern, filter).glob();
2025    }
2026
2027    /**
2028     * Copy file from src to dest. See
2029     * {@link #copy(Path, Path, boolean, boolean)}
2030     */
2031    public boolean copy(final Path src, final Path dst)
2032        throws AccessControlException, FileAlreadyExistsException,
2033        FileNotFoundException, ParentNotDirectoryException,
2034        UnsupportedFileSystemException, IOException {
2035      return copy(src, dst, false, false);
2036    }
2037    
2038    /**
2039     * Copy from src to dst, optionally deleting src and overwriting dst.
2040     * @param src
2041     * @param dst
2042     * @param deleteSource - delete src if true
2043     * @param overwrite  overwrite dst if true; throw IOException if dst exists
2044     *         and overwrite is false.
2045     *
2046     * @return true if copy is successful
2047     *
2048     * @throws AccessControlException If access is denied
2049     * @throws FileAlreadyExistsException If <code>dst</code> already exists
2050     * @throws FileNotFoundException If <code>src</code> does not exist
2051     * @throws ParentNotDirectoryException If parent of <code>dst</code> is not
2052     *           a directory
2053     * @throws UnsupportedFileSystemException If file system for 
2054     *         <code>src</code> or <code>dst</code> is not supported
2055     * @throws IOException If an I/O error occurred
2056     * 
2057     * Exceptions applicable to file systems accessed over RPC:
2058     * @throws RpcClientException If an exception occurred in the RPC client
2059     * @throws RpcServerException If an exception occurred in the RPC server
2060     * @throws UnexpectedServerException If server implementation throws 
2061     *           undeclared exception to RPC server
2062     * 
2063     * RuntimeExceptions:
2064     * @throws InvalidPathException If path <code>dst</code> is invalid
2065     */
2066    public boolean copy(final Path src, final Path dst, boolean deleteSource,
2067        boolean overwrite) throws AccessControlException,
2068        FileAlreadyExistsException, FileNotFoundException,
2069        ParentNotDirectoryException, UnsupportedFileSystemException, 
2070        IOException {
2071      src.checkNotSchemeWithRelative();
2072      dst.checkNotSchemeWithRelative();
2073      Path qSrc = makeQualified(src);
2074      Path qDst = makeQualified(dst);
2075      checkDest(qSrc.getName(), qDst, overwrite);
2076      FileStatus fs = FileContext.this.getFileStatus(qSrc);
2077      if (fs.isDirectory()) {
2078        checkDependencies(qSrc, qDst);
2079        mkdir(qDst, FsPermission.getDirDefault(), true);
2080        FileStatus[] contents = listStatus(qSrc);
2081        for (FileStatus content : contents) {
2082          copy(makeQualified(content.getPath()), makeQualified(new Path(qDst,
2083              content.getPath().getName())), deleteSource, overwrite);
2084        }
2085      } else {
2086        InputStream in=null;
2087        OutputStream out = null;
2088        try {
2089          in = open(qSrc);
2090          EnumSet<CreateFlag> createFlag = overwrite ? EnumSet.of(
2091              CreateFlag.CREATE, CreateFlag.OVERWRITE) : 
2092                EnumSet.of(CreateFlag.CREATE);
2093          out = create(qDst, createFlag);
2094          IOUtils.copyBytes(in, out, conf, true);
2095        } finally {
2096          IOUtils.closeStream(out);
2097          IOUtils.closeStream(in);
2098        }
2099      }
2100      if (deleteSource) {
2101        return delete(qSrc, true);
2102      } else {
2103        return true;
2104      }
2105    }
2106  }
2107
2108  /**
2109   * Check if copying srcName to dst would overwrite an existing 
2110   * file or directory.
2111   * @param srcName File or directory to be copied.
2112   * @param dst Destination to copy srcName to.
2113   * @param overwrite Whether it's ok to overwrite an existing file. 
2114   * @throws AccessControlException If access is denied.
2115   * @throws IOException If dst is an existing directory, or dst is an 
2116   * existing file and the overwrite option is not passed.
2117   */
2118  private void checkDest(String srcName, Path dst, boolean overwrite)
2119      throws AccessControlException, IOException {
2120    try {
2121      FileStatus dstFs = getFileStatus(dst);
2122      if (dstFs.isDirectory()) {
2123        if (null == srcName) {
2124          throw new IOException("Target " + dst + " is a directory");
2125        }
2126        // Recurse to check if dst/srcName exists.
2127        checkDest(null, new Path(dst, srcName), overwrite);
2128      } else if (!overwrite) {
2129        throw new IOException("Target " + new Path(dst, srcName)
2130            + " already exists");
2131      }
2132    } catch (FileNotFoundException e) {
2133      // dst does not exist - OK to copy.
2134    }
2135  }
2136   
2137  //
2138  // If the destination is a subdirectory of the source, then
2139  // generate exception
2140  //
2141  private static void checkDependencies(Path qualSrc, Path qualDst)
2142    throws IOException {
2143    if (isSameFS(qualSrc, qualDst)) {
2144      String srcq = qualSrc.toString() + Path.SEPARATOR;
2145      String dstq = qualDst.toString() + Path.SEPARATOR;
2146      if (dstq.startsWith(srcq)) {
2147        if (srcq.length() == dstq.length()) {
2148          throw new IOException("Cannot copy " + qualSrc + " to itself.");
2149        } else {
2150          throw new IOException("Cannot copy " + qualSrc +
2151                             " to its subdirectory " + qualDst);
2152        }
2153      }
2154    }
2155  }
2156  
2157  /**
2158   * Are qualSrc and qualDst of the same file system?
2159   * @param qualPath1 - fully qualified path
2160   * @param qualPath2 - fully qualified path
2161   * @return
2162   */
2163  private static boolean isSameFS(Path qualPath1, Path qualPath2) {
2164    URI srcUri = qualPath1.toUri();
2165    URI dstUri = qualPath2.toUri();
2166    return (srcUri.getScheme().equals(dstUri.getScheme()) && 
2167        !(srcUri.getAuthority() != null && dstUri.getAuthority() != null && srcUri
2168        .getAuthority().equals(dstUri.getAuthority())));
2169  }
2170
2171  /**
2172   * Deletes all the paths in deleteOnExit on JVM shutdown.
2173   */
2174  static class FileContextFinalizer implements Runnable {
2175    @Override
2176    public synchronized void run() {
2177      processDeleteOnExit();
2178    }
2179  }
2180
2181  /**
2182   * Resolves all symbolic links in the specified path.
2183   * Returns the new path object.
2184   */
2185  protected Path resolve(final Path f) throws FileNotFoundException,
2186      UnresolvedLinkException, AccessControlException, IOException {
2187    return new FSLinkResolver<Path>() {
2188      @Override
2189      public Path next(final AbstractFileSystem fs, final Path p) 
2190        throws IOException, UnresolvedLinkException {
2191        return fs.resolvePath(p);
2192      }
2193    }.resolve(this, f);
2194  }
2195
2196  /**
2197   * Resolves all symbolic links in the specified path leading up 
2198   * to, but not including the final path component.
2199   * @param f path to resolve
2200   * @return the new path object.
2201   */
2202  protected Path resolveIntermediate(final Path f) throws IOException {
2203    return new FSLinkResolver<FileStatus>() {
2204      @Override
2205      public FileStatus next(final AbstractFileSystem fs, final Path p) 
2206        throws IOException, UnresolvedLinkException {
2207        return fs.getFileLinkStatus(p);
2208      }
2209    }.resolve(this, f).getPath();
2210  }
2211
2212  /**
2213   * Returns the list of AbstractFileSystems accessed in the path. The list may
2214   * contain more than one AbstractFileSystems objects in case of symlinks.
2215   * 
2216   * @param f
2217   *          Path which needs to be resolved
2218   * @return List of AbstractFileSystems accessed in the path
2219   * @throws IOException
2220   */
2221  Set<AbstractFileSystem> resolveAbstractFileSystems(final Path f)
2222      throws IOException {
2223    final Path absF = fixRelativePart(f);
2224    final HashSet<AbstractFileSystem> result 
2225      = new HashSet<AbstractFileSystem>();
2226    new FSLinkResolver<Void>() {
2227      @Override
2228      public Void next(final AbstractFileSystem fs, final Path p)
2229          throws IOException, UnresolvedLinkException {
2230        result.add(fs);
2231        fs.getFileStatus(p);
2232        return null;
2233      }
2234    }.resolve(this, absF);
2235    return result;
2236  }
2237
2238  /**
2239   * Get the statistics for a particular file system
2240   * 
2241   * @param uri
2242   *          the uri to lookup the statistics. Only scheme and authority part
2243   *          of the uri are used as the key to store and lookup.
2244   * @return a statistics object
2245   */
2246  public static Statistics getStatistics(URI uri) {
2247    return AbstractFileSystem.getStatistics(uri);
2248  }
2249
2250  /**
2251   * Clears all the statistics stored in AbstractFileSystem, for all the file
2252   * systems.
2253   */
2254  public static void clearStatistics() {
2255    AbstractFileSystem.clearStatistics();
2256  }
2257
2258  /**
2259   * Prints the statistics to standard output. File System is identified by the
2260   * scheme and authority.
2261   */
2262  public static void printStatistics() {
2263    AbstractFileSystem.printStatistics();
2264  }
2265
2266  /**
2267   * @return Map of uri and statistics for each filesystem instantiated. The uri
2268   *         consists of scheme and authority for the filesystem.
2269   */
2270  public static Map<URI, Statistics> getAllStatistics() {
2271    return AbstractFileSystem.getAllStatistics();
2272  }
2273  
2274  /**
2275   * Get delegation tokens for the file systems accessed for a given
2276   * path.
2277   * @param p Path for which delegations tokens are requested.
2278   * @param renewer the account name that is allowed to renew the token.
2279   * @return List of delegation tokens.
2280   * @throws IOException
2281   */
2282  @InterfaceAudience.LimitedPrivate( { "HDFS", "MapReduce" })
2283  public List<Token<?>> getDelegationTokens(
2284      Path p, String renewer) throws IOException {
2285    Set<AbstractFileSystem> afsSet = resolveAbstractFileSystems(p);
2286    List<Token<?>> tokenList = 
2287        new ArrayList<Token<?>>();
2288    for (AbstractFileSystem afs : afsSet) {
2289      List<Token<?>> afsTokens = afs.getDelegationTokens(renewer);
2290      tokenList.addAll(afsTokens);
2291    }
2292    return tokenList;
2293  }
2294
2295  /**
2296   * Modifies ACL entries of files and directories.  This method can add new ACL
2297   * entries or modify the permissions on existing ACL entries.  All existing
2298   * ACL entries that are not specified in this call are retained without
2299   * changes.  (Modifications are merged into the current ACL.)
2300   *
2301   * @param path Path to modify
2302   * @param aclSpec List<AclEntry> describing modifications
2303   * @throws IOException if an ACL could not be modified
2304   */
2305  public void modifyAclEntries(final Path path, final List<AclEntry> aclSpec)
2306      throws IOException {
2307    Path absF = fixRelativePart(path);
2308    new FSLinkResolver<Void>() {
2309      @Override
2310      public Void next(final AbstractFileSystem fs, final Path p)
2311          throws IOException {
2312        fs.modifyAclEntries(p, aclSpec);
2313        return null;
2314      }
2315    }.resolve(this, absF);
2316  }
2317
2318  /**
2319   * Removes ACL entries from files and directories.  Other ACL entries are
2320   * retained.
2321   *
2322   * @param path Path to modify
2323   * @param aclSpec List<AclEntry> describing entries to remove
2324   * @throws IOException if an ACL could not be modified
2325   */
2326  public void removeAclEntries(final Path path, final List<AclEntry> aclSpec)
2327      throws IOException {
2328    Path absF = fixRelativePart(path);
2329    new FSLinkResolver<Void>() {
2330      @Override
2331      public Void next(final AbstractFileSystem fs, final Path p)
2332          throws IOException {
2333        fs.removeAclEntries(p, aclSpec);
2334        return null;
2335      }
2336    }.resolve(this, absF);
2337  }
2338
2339  /**
2340   * Removes all default ACL entries from files and directories.
2341   *
2342   * @param path Path to modify
2343   * @throws IOException if an ACL could not be modified
2344   */
2345  public void removeDefaultAcl(Path path)
2346      throws IOException {
2347    Path absF = fixRelativePart(path);
2348    new FSLinkResolver<Void>() {
2349      @Override
2350      public Void next(final AbstractFileSystem fs, final Path p)
2351          throws IOException {
2352        fs.removeDefaultAcl(p);
2353        return null;
2354      }
2355    }.resolve(this, absF);
2356  }
2357
2358  /**
2359   * Removes all but the base ACL entries of files and directories.  The entries
2360   * for user, group, and others are retained for compatibility with permission
2361   * bits.
2362   *
2363   * @param path Path to modify
2364   * @throws IOException if an ACL could not be removed
2365   */
2366  public void removeAcl(Path path) throws IOException {
2367    Path absF = fixRelativePart(path);
2368    new FSLinkResolver<Void>() {
2369      @Override
2370      public Void next(final AbstractFileSystem fs, final Path p)
2371          throws IOException {
2372        fs.removeAcl(p);
2373        return null;
2374      }
2375    }.resolve(this, absF);
2376  }
2377
2378  /**
2379   * Fully replaces ACL of files and directories, discarding all existing
2380   * entries.
2381   *
2382   * @param path Path to modify
2383   * @param aclSpec List<AclEntry> describing modifications, must include entries
2384   *   for user, group, and others for compatibility with permission bits.
2385   * @throws IOException if an ACL could not be modified
2386   */
2387  public void setAcl(Path path, final List<AclEntry> aclSpec)
2388      throws IOException {
2389    Path absF = fixRelativePart(path);
2390    new FSLinkResolver<Void>() {
2391      @Override
2392      public Void next(final AbstractFileSystem fs, final Path p)
2393          throws IOException {
2394        fs.setAcl(p, aclSpec);
2395        return null;
2396      }
2397    }.resolve(this, absF);
2398  }
2399
2400  /**
2401   * Gets the ACLs of files and directories.
2402   *
2403   * @param path Path to get
2404   * @return RemoteIterator<AclStatus> which returns each AclStatus
2405   * @throws IOException if an ACL could not be read
2406   */
2407  public AclStatus getAclStatus(Path path) throws IOException {
2408    Path absF = fixRelativePart(path);
2409    return new FSLinkResolver<AclStatus>() {
2410      @Override
2411      public AclStatus next(final AbstractFileSystem fs, final Path p)
2412          throws IOException {
2413        return fs.getAclStatus(p);
2414      }
2415    }.resolve(this, absF);
2416  }
2417
2418  /**
2419   * Set an xattr of a file or directory.
2420   * The name must be prefixed with the namespace followed by ".". For example,
2421   * "user.attr".
2422   * <p/>
2423   * Refer to the HDFS extended attributes user documentation for details.
2424   *
2425   * @param path Path to modify
2426   * @param name xattr name.
2427   * @param value xattr value.
2428   * @throws IOException
2429   */
2430  public void setXAttr(Path path, String name, byte[] value)
2431      throws IOException {
2432    setXAttr(path, name, value, EnumSet.of(XAttrSetFlag.CREATE,
2433        XAttrSetFlag.REPLACE));
2434  }
2435
2436  /**
2437   * Set an xattr of a file or directory.
2438   * The name must be prefixed with the namespace followed by ".". For example,
2439   * "user.attr".
2440   * <p/>
2441   * Refer to the HDFS extended attributes user documentation for details.
2442   *
2443   * @param path Path to modify
2444   * @param name xattr name.
2445   * @param value xattr value.
2446   * @param flag xattr set flag
2447   * @throws IOException
2448   */
2449  public void setXAttr(Path path, final String name, final byte[] value,
2450      final EnumSet<XAttrSetFlag> flag) throws IOException {
2451    final Path absF = fixRelativePart(path);
2452    new FSLinkResolver<Void>() {
2453      @Override
2454      public Void next(final AbstractFileSystem fs, final Path p)
2455          throws IOException {
2456        fs.setXAttr(p, name, value, flag);
2457        return null;
2458      }
2459    }.resolve(this, absF);
2460  }
2461
2462  /**
2463   * Get an xattr for a file or directory.
2464   * The name must be prefixed with the namespace followed by ".". For example,
2465   * "user.attr".
2466   * <p/>
2467   * Refer to the HDFS extended attributes user documentation for details.
2468   *
2469   * @param path Path to get extended attribute
2470   * @param name xattr name.
2471   * @return byte[] xattr value.
2472   * @throws IOException
2473   */
2474  public byte[] getXAttr(Path path, final String name) throws IOException {
2475    final Path absF = fixRelativePart(path);
2476    return new FSLinkResolver<byte[]>() {
2477      @Override
2478      public byte[] next(final AbstractFileSystem fs, final Path p)
2479          throws IOException {
2480        return fs.getXAttr(p, name);
2481      }
2482    }.resolve(this, absF);
2483  }
2484
2485  /**
2486   * Get all of the xattrs for a file or directory.
2487   * Only those xattrs for which the logged-in user has permissions to view
2488   * are returned.
2489   * <p/>
2490   * Refer to the HDFS extended attributes user documentation for details.
2491   *
2492   * @param path Path to get extended attributes
2493   * @return Map<String, byte[]> describing the XAttrs of the file or directory
2494   * @throws IOException
2495   */
2496  public Map<String, byte[]> getXAttrs(Path path) throws IOException {
2497    final Path absF = fixRelativePart(path);
2498    return new FSLinkResolver<Map<String, byte[]>>() {
2499      @Override
2500      public Map<String, byte[]> next(final AbstractFileSystem fs, final Path p)
2501          throws IOException {
2502        return fs.getXAttrs(p);
2503      }
2504    }.resolve(this, absF);
2505  }
2506
2507  /**
2508   * Get all of the xattrs for a file or directory.
2509   * Only those xattrs for which the logged-in user has permissions to view
2510   * are returned.
2511   * <p/>
2512   * Refer to the HDFS extended attributes user documentation for details.
2513   *
2514   * @param path Path to get extended attributes
2515   * @param names XAttr names.
2516   * @return Map<String, byte[]> describing the XAttrs of the file or directory
2517   * @throws IOException
2518   */
2519  public Map<String, byte[]> getXAttrs(Path path, final List<String> names)
2520      throws IOException {
2521    final Path absF = fixRelativePart(path);
2522    return new FSLinkResolver<Map<String, byte[]>>() {
2523      @Override
2524      public Map<String, byte[]> next(final AbstractFileSystem fs, final Path p)
2525          throws IOException {
2526        return fs.getXAttrs(p, names);
2527      }
2528    }.resolve(this, absF);
2529  }
2530
2531  /**
2532   * Remove an xattr of a file or directory.
2533   * The name must be prefixed with the namespace followed by ".". For example,
2534   * "user.attr".
2535   * <p/>
2536   * Refer to the HDFS extended attributes user documentation for details.
2537   *
2538   * @param path Path to remove extended attribute
2539   * @param name xattr name
2540   * @throws IOException
2541   */
2542  public void removeXAttr(Path path, final String name) throws IOException {
2543    final Path absF = fixRelativePart(path);
2544    new FSLinkResolver<Void>() {
2545      @Override
2546      public Void next(final AbstractFileSystem fs, final Path p)
2547          throws IOException {
2548        fs.removeXAttr(p, name);
2549        return null;
2550      }
2551    }.resolve(this, absF);
2552  }
2553
2554  /**
2555   * Get all of the xattr names for a file or directory.
2556   * Only those xattr names which the logged-in user has permissions to view
2557   * are returned.
2558   * <p/>
2559   * Refer to the HDFS extended attributes user documentation for details.
2560   *
2561   * @param path Path to get extended attributes
2562   * @return List<String> of the XAttr names of the file or directory
2563   * @throws IOException
2564   */
2565  public List<String> listXAttrs(Path path) throws IOException {
2566    final Path absF = fixRelativePart(path);
2567    return new FSLinkResolver<List<String>>() {
2568      @Override
2569      public List<String> next(final AbstractFileSystem fs, final Path p)
2570          throws IOException {
2571        return fs.listXAttrs(p);
2572      }
2573    }.resolve(this, absF);
2574  }
2575
2576  /**
2577   * Create a snapshot with a default name.
2578   *
2579   * @param path The directory where snapshots will be taken.
2580   * @return the snapshot path.
2581   *
2582   * @throws IOException If an I/O error occurred
2583   *
2584   * <p>Exceptions applicable to file systems accessed over RPC:
2585   * @throws RpcClientException If an exception occurred in the RPC client
2586   * @throws RpcServerException If an exception occurred in the RPC server
2587   * @throws UnexpectedServerException If server implementation throws
2588   *           undeclared exception to RPC server
2589   */
2590  public final Path createSnapshot(Path path) throws IOException {
2591    return createSnapshot(path, null);
2592  }
2593
2594  /**
2595   * Create a snapshot.
2596   *
2597   * @param path The directory where snapshots will be taken.
2598   * @param snapshotName The name of the snapshot
2599   * @return the snapshot path.
2600   *
2601   * @throws IOException If an I/O error occurred
2602   *
2603   * <p>Exceptions applicable to file systems accessed over RPC:
2604   * @throws RpcClientException If an exception occurred in the RPC client
2605   * @throws RpcServerException If an exception occurred in the RPC server
2606   * @throws UnexpectedServerException If server implementation throws
2607   *           undeclared exception to RPC server
2608   */
2609  public Path createSnapshot(final Path path, final String snapshotName)
2610      throws IOException {
2611    final Path absF = fixRelativePart(path);
2612    return new FSLinkResolver<Path>() {
2613
2614      @Override
2615      public Path next(final AbstractFileSystem fs, final Path p)
2616          throws IOException {
2617        return fs.createSnapshot(p, snapshotName);
2618      }
2619    }.resolve(this, absF);
2620  }
2621
2622  /**
2623   * Rename a snapshot.
2624   *
2625   * @param path The directory path where the snapshot was taken
2626   * @param snapshotOldName Old name of the snapshot
2627   * @param snapshotNewName New name of the snapshot
2628   *
2629   * @throws IOException If an I/O error occurred
2630   *
2631   * <p>Exceptions applicable to file systems accessed over RPC:
2632   * @throws RpcClientException If an exception occurred in the RPC client
2633   * @throws RpcServerException If an exception occurred in the RPC server
2634   * @throws UnexpectedServerException If server implementation throws
2635   *           undeclared exception to RPC server
2636   */
2637  public void renameSnapshot(final Path path, final String snapshotOldName,
2638      final String snapshotNewName) throws IOException {
2639    final Path absF = fixRelativePart(path);
2640    new FSLinkResolver<Void>() {
2641      @Override
2642      public Void next(final AbstractFileSystem fs, final Path p)
2643          throws IOException {
2644        fs.renameSnapshot(p, snapshotOldName, snapshotNewName);
2645        return null;
2646      }
2647    }.resolve(this, absF);
2648  }
2649
2650  /**
2651   * Delete a snapshot of a directory.
2652   *
2653   * @param path The directory that the to-be-deleted snapshot belongs to
2654   * @param snapshotName The name of the snapshot
2655   *
2656   * @throws IOException If an I/O error occurred
2657   *
2658   * <p>Exceptions applicable to file systems accessed over RPC:
2659   * @throws RpcClientException If an exception occurred in the RPC client
2660   * @throws RpcServerException If an exception occurred in the RPC server
2661   * @throws UnexpectedServerException If server implementation throws
2662   *           undeclared exception to RPC server
2663   */
2664  public void deleteSnapshot(final Path path, final String snapshotName)
2665      throws IOException {
2666    final Path absF = fixRelativePart(path);
2667    new FSLinkResolver<Void>() {
2668      @Override
2669      public Void next(final AbstractFileSystem fs, final Path p)
2670          throws IOException {
2671        fs.deleteSnapshot(p, snapshotName);
2672        return null;
2673      }
2674    }.resolve(this, absF);
2675  }
2676
2677  /**
2678   * Set the storage policy for a given file or directory.
2679   *
2680   * @param path file or directory path.
2681   * @param policyName the name of the target storage policy. The list
2682   *                   of supported Storage policies can be retrieved
2683   *                   via {@link #getAllStoragePolicies}.
2684   */
2685  public void setStoragePolicy(final Path path, final String policyName)
2686      throws IOException {
2687    final Path absF = fixRelativePart(path);
2688    new FSLinkResolver<Void>() {
2689      @Override
2690      public Void next(final AbstractFileSystem fs, final Path p)
2691          throws IOException {
2692        fs.setStoragePolicy(path, policyName);
2693        return null;
2694      }
2695    }.resolve(this, absF);
2696  }
2697
2698  /**
2699   * Query the effective storage policy ID for the given file or directory.
2700   *
2701   * @param src file or directory path.
2702   * @return storage policy for give file.
2703   * @throws IOException
2704   */
2705  public BlockStoragePolicySpi getStoragePolicy(Path path) throws IOException {
2706    final Path absF = fixRelativePart(path);
2707    return new FSLinkResolver<BlockStoragePolicySpi>() {
2708      @Override
2709      public BlockStoragePolicySpi next(final AbstractFileSystem fs,
2710          final Path p)
2711          throws IOException {
2712        return fs.getStoragePolicy(p);
2713      }
2714    }.resolve(this, absF);
2715  }
2716
2717  /**
2718   * Retrieve all the storage policies supported by this file system.
2719   *
2720   * @return all storage policies supported by this filesystem.
2721   * @throws IOException
2722   */
2723  public Collection<? extends BlockStoragePolicySpi> getAllStoragePolicies()
2724      throws IOException {
2725    return defaultFS.getAllStoragePolicies();
2726  }
2727
2728  Tracer getTracer() {
2729    return tracer;
2730  }
2731}