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