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