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    
021    import java.io.FileNotFoundException;
022    import java.io.IOException;
023    import java.lang.reflect.Constructor;
024    import java.net.URI;
025    import java.net.URISyntaxException;
026    import java.util.ArrayList;
027    import java.util.EnumSet;
028    import java.util.HashMap;
029    import java.util.List;
030    import java.util.Map;
031    import java.util.NoSuchElementException;
032    import java.util.StringTokenizer;
033    import java.util.concurrent.ConcurrentHashMap;
034    
035    import org.apache.commons.logging.Log;
036    import org.apache.commons.logging.LogFactory;
037    import org.apache.hadoop.HadoopIllegalArgumentException;
038    import org.apache.hadoop.classification.InterfaceAudience;
039    import org.apache.hadoop.classification.InterfaceStability;
040    import org.apache.hadoop.conf.Configuration;
041    import org.apache.hadoop.fs.FileSystem.Statistics;
042    import org.apache.hadoop.fs.Options.CreateOpts;
043    import org.apache.hadoop.fs.Options.Rename;
044    import org.apache.hadoop.fs.permission.FsPermission;
045    import org.apache.hadoop.fs.InvalidPathException;
046    import org.apache.hadoop.security.AccessControlException;
047    import org.apache.hadoop.security.SecurityUtil;
048    import org.apache.hadoop.security.token.Token;
049    import org.apache.hadoop.util.Progressable;
050    
051    /**
052     * This class provides an interface for implementors of a Hadoop file system
053     * (analogous to the VFS of Unix). Applications do not access this class;
054     * instead they access files across all file systems using {@link FileContext}.
055     * 
056     * Pathnames passed to AbstractFileSystem can be fully qualified URI that
057     * matches the "this" file system (ie same scheme and authority) 
058     * or a Slash-relative name that is assumed to be relative
059     * to the root of the "this" file system .
060     */
061    @InterfaceAudience.Public
062    @InterfaceStability.Evolving /*Evolving for a release,to be changed to Stable */
063    public abstract class AbstractFileSystem {
064      static final Log LOG = LogFactory.getLog(AbstractFileSystem.class);
065    
066      /** Recording statistics per a file system class. */
067      private static final Map<URI, Statistics> 
068          STATISTICS_TABLE = new HashMap<URI, Statistics>();
069      
070      /** Cache of constructors for each file system class. */
071      private static final Map<Class<?>, Constructor<?>> CONSTRUCTOR_CACHE = 
072        new ConcurrentHashMap<Class<?>, Constructor<?>>();
073      
074      private static final Class<?>[] URI_CONFIG_ARGS = 
075        new Class[]{URI.class, Configuration.class};
076      
077      /** The statistics for this file system. */
078      protected Statistics statistics;
079      
080      private final URI myUri;
081      
082      public Statistics getStatistics() {
083        return statistics;
084      }
085      
086      /**
087       * Prohibits names which contain a ".", "..", ":" or "/" 
088       */
089      private static boolean isValidName(String src) {
090        // Check for ".." "." ":" "/"
091        StringTokenizer tokens = new StringTokenizer(src, Path.SEPARATOR);
092        while(tokens.hasMoreTokens()) {
093          String element = tokens.nextToken();
094          if (element.equals("target/generated-sources") ||
095              element.equals(".")  ||
096              (element.indexOf(":") >= 0)) {
097            return false;
098          }
099        }
100        return true;
101      }
102      
103      /** 
104       * Create an object for the given class and initialize it from conf.
105       * @param theClass class of which an object is created
106       * @param conf Configuration
107       * @return a new object
108       */
109      @SuppressWarnings("unchecked")
110      static <T> T newInstance(Class<T> theClass,
111        URI uri, Configuration conf) {
112        T result;
113        try {
114          Constructor<T> meth = (Constructor<T>) CONSTRUCTOR_CACHE.get(theClass);
115          if (meth == null) {
116            meth = theClass.getDeclaredConstructor(URI_CONFIG_ARGS);
117            meth.setAccessible(true);
118            CONSTRUCTOR_CACHE.put(theClass, meth);
119          }
120          result = meth.newInstance(uri, conf);
121        } catch (Exception e) {
122          throw new RuntimeException(e);
123        }
124        return result;
125      }
126      
127      /**
128       * Create a file system instance for the specified uri using the conf. The
129       * conf is used to find the class name that implements the file system. The
130       * conf is also passed to the file system for its configuration.
131       *
132       * @param uri URI of the file system
133       * @param conf Configuration for the file system
134       * 
135       * @return Returns the file system for the given URI
136       *
137       * @throws UnsupportedFileSystemException file system for <code>uri</code> is
138       *           not found
139       */
140      public static AbstractFileSystem createFileSystem(URI uri, Configuration conf)
141          throws UnsupportedFileSystemException {
142        Class<?> clazz = conf.getClass("fs.AbstractFileSystem." + 
143                                    uri.getScheme() + ".impl", null);
144        if (clazz == null) {
145          throw new UnsupportedFileSystemException(
146              "No AbstractFileSystem for scheme: " + uri.getScheme());
147        }
148        return (AbstractFileSystem) newInstance(clazz, uri, conf);
149      }
150    
151      /**
152       * Get the statistics for a particular file system.
153       * 
154       * @param uri
155       *          used as key to lookup STATISTICS_TABLE. Only scheme and authority
156       *          part of the uri are used.
157       * @return a statistics object
158       */
159      protected static synchronized Statistics getStatistics(URI uri) {
160        String scheme = uri.getScheme();
161        if (scheme == null) {
162          throw new IllegalArgumentException("Scheme not defined in the uri: "
163              + uri);
164        }
165        URI baseUri = getBaseUri(uri);
166        Statistics result = STATISTICS_TABLE.get(baseUri);
167        if (result == null) {
168          result = new Statistics(scheme);
169          STATISTICS_TABLE.put(baseUri, result);
170        }
171        return result;
172      }
173      
174      private static URI getBaseUri(URI uri) {
175        String scheme = uri.getScheme();
176        String authority = uri.getAuthority();
177        String baseUriString = scheme + "://";
178        if (authority != null) {
179          baseUriString = baseUriString + authority;
180        } else {
181          baseUriString = baseUriString + "/";
182        }
183        return URI.create(baseUriString);
184      }
185      
186      public static synchronized void clearStatistics() {
187        for(Statistics stat: STATISTICS_TABLE.values()) {
188          stat.reset();
189        }
190      }
191    
192      /**
193       * Prints statistics for all file systems.
194       */
195      public static synchronized void printStatistics() {
196        for (Map.Entry<URI, Statistics> pair : STATISTICS_TABLE.entrySet()) {
197          System.out.println("  FileSystem " + pair.getKey().getScheme() + "://"
198              + pair.getKey().getAuthority() + ": " + pair.getValue());
199        }
200      }
201      
202      protected static synchronized Map<URI, Statistics> getAllStatistics() {
203        Map<URI, Statistics> statsMap = new HashMap<URI, Statistics>(
204            STATISTICS_TABLE.size());
205        for (Map.Entry<URI, Statistics> pair : STATISTICS_TABLE.entrySet()) {
206          URI key = pair.getKey();
207          Statistics value = pair.getValue();
208          Statistics newStatsObj = new Statistics(value);
209          statsMap.put(URI.create(key.toString()), newStatsObj);
210        }
211        return statsMap;
212      }
213    
214      /**
215       * The main factory method for creating a file system. Get a file system for
216       * the URI's scheme and authority. The scheme of the <code>uri</code>
217       * determines a configuration property name,
218       * <tt>fs.AbstractFileSystem.<i>scheme</i>.impl</tt> whose value names the
219       * AbstractFileSystem class.
220       * 
221       * The entire URI and conf is passed to the AbstractFileSystem factory method.
222       * 
223       * @param uri for the file system to be created.
224       * @param conf which is passed to the file system impl.
225       * 
226       * @return file system for the given URI.
227       * 
228       * @throws UnsupportedFileSystemException if the file system for
229       *           <code>uri</code> is not supported.
230       */
231      public static AbstractFileSystem get(final URI uri, final Configuration conf)
232          throws UnsupportedFileSystemException {
233        return createFileSystem(uri, conf);
234      }
235    
236      /**
237       * Constructor to be called by subclasses.
238       * 
239       * @param uri for this file system.
240       * @param supportedScheme the scheme supported by the implementor
241       * @param authorityNeeded if true then theURI must have authority, if false
242       *          then the URI must have null authority.
243       *
244       * @throws URISyntaxException <code>uri</code> has syntax error
245       */
246      public AbstractFileSystem(final URI uri, final String supportedScheme,
247          final boolean authorityNeeded, final int defaultPort)
248          throws URISyntaxException {
249        myUri = getUri(uri, supportedScheme, authorityNeeded, defaultPort);
250        statistics = getStatistics(uri); 
251      }
252      
253      /**
254       * Check that the Uri's scheme matches
255       * @param uri
256       * @param supportedScheme
257       */
258      public void checkScheme(URI uri, String supportedScheme) {
259        String scheme = uri.getScheme();
260        if (scheme == null) {
261          throw new HadoopIllegalArgumentException("Uri without scheme: " + uri);
262        }
263        if (!scheme.equals(supportedScheme)) {
264          throw new HadoopIllegalArgumentException("Uri scheme " + uri
265              + " does not match the scheme " + supportedScheme);
266        }
267      }
268    
269      /**
270       * Get the URI for the file system based on the given URI. The path, query
271       * part of the given URI is stripped out and default file system port is used
272       * to form the URI.
273       * 
274       * @param uri FileSystem URI.
275       * @param authorityNeeded if true authority cannot be null in the URI. If
276       *          false authority must be null.
277       * @param defaultPort default port to use if port is not specified in the URI.
278       * 
279       * @return URI of the file system
280       * 
281       * @throws URISyntaxException <code>uri</code> has syntax error
282       */
283      private URI getUri(URI uri, String supportedScheme,
284          boolean authorityNeeded, int defaultPort) throws URISyntaxException {
285        checkScheme(uri, supportedScheme);
286        // A file system implementation that requires authority must always
287        // specify default port
288        if (defaultPort < 0 && authorityNeeded) {
289          throw new HadoopIllegalArgumentException(
290              "FileSystem implementation error -  default port " + defaultPort
291                  + " is not valid");
292        }
293        String authority = uri.getAuthority();
294        if (authority == null) {
295           if (authorityNeeded) {
296             throw new HadoopIllegalArgumentException("Uri without authority: " + uri);
297           } else {
298             return new URI(supportedScheme + ":///");
299           }   
300        }
301        // authority is non null  - AuthorityNeeded may be true or false.
302        int port = uri.getPort();
303        port = (port == -1 ? defaultPort : port);
304        if (port == -1) { // no port supplied and default port is not specified
305          return new URI(supportedScheme, authority, "/", null);
306        }
307        return new URI(supportedScheme + "://" + uri.getHost() + ":" + port);
308      }
309      
310      /**
311       * The default port of this file system.
312       * 
313       * @return default port of this file system's Uri scheme
314       *         A uri with a port of -1 => default port;
315       */
316      public abstract int getUriDefaultPort();
317    
318      /**
319       * Returns a URI whose scheme and authority identify this FileSystem.
320       * 
321       * @return the uri of this file system.
322       */
323      public URI getUri() {
324        return myUri;
325      }
326      
327      /**
328       * Check that a Path belongs to this FileSystem.
329       * 
330       * If the path is fully qualified URI, then its scheme and authority
331       * matches that of this file system. Otherwise the path must be 
332       * slash-relative name.
333       * 
334       * @throws InvalidPathException if the path is invalid
335       */
336      public void checkPath(Path path) {
337        URI uri = path.toUri();
338        String thatScheme = uri.getScheme();
339        String thatAuthority = uri.getAuthority();
340        if (thatScheme == null) {
341          if (thatAuthority == null) {
342            if (path.isUriPathAbsolute()) {
343              return;
344            }
345            throw new InvalidPathException("relative paths not allowed:" + 
346                path);
347          } else {
348            throw new InvalidPathException(
349                "Path without scheme with non-null authority:" + path);
350          }
351        }
352        String thisScheme = this.getUri().getScheme();
353        String thisHost = this.getUri().getHost();
354        String thatHost = uri.getHost();
355        
356        // Schemes and hosts must match.
357        // Allow for null Authority for file:///
358        if (!thisScheme.equalsIgnoreCase(thatScheme) ||
359           (thisHost != null && 
360                !thisHost.equalsIgnoreCase(thatHost)) ||
361           (thisHost == null && thatHost != null)) {
362          throw new InvalidPathException("Wrong FS: " + path + ", expected: "
363              + this.getUri());
364        }
365        
366        // Ports must match, unless this FS instance is using the default port, in
367        // which case the port may be omitted from the given URI
368        int thisPort = this.getUri().getPort();
369        int thatPort = uri.getPort();
370        if (thatPort == -1) { // -1 => defaultPort of Uri scheme
371          thatPort = this.getUriDefaultPort();
372        }
373        if (thisPort != thatPort) {
374          throw new InvalidPathException("Wrong FS: " + path + ", expected: "
375              + this.getUri());
376        }
377      }
378      
379      /**
380       * Get the path-part of a pathname. Checks that URI matches this file system
381       * and that the path-part is a valid name.
382       * 
383       * @param p path
384       * 
385       * @return path-part of the Path p
386       */
387      public String getUriPath(final Path p) {
388        checkPath(p);
389        String s = p.toUri().getPath();
390        if (!isValidName(s)) {
391          throw new InvalidPathException("Path part " + s + " from URI " + p
392              + " is not a valid filename.");
393        }
394        return s;
395      }
396      
397      /**
398       * Make the path fully qualified to this file system
399       * @param path
400       * @return the qualified path
401       */
402      public Path makeQualified(Path path) {
403        checkPath(path);
404        return path.makeQualified(this.getUri(), null);
405      }
406      
407      /**
408       * Some file systems like LocalFileSystem have an initial workingDir
409       * that is used as the starting workingDir. For other file systems
410       * like HDFS there is no built in notion of an initial workingDir.
411       * 
412       * @return the initial workingDir if the file system has such a notion
413       *         otherwise return a null.
414       */
415      public Path getInitialWorkingDirectory() {
416        return null;
417      }
418      
419      /** 
420       * Return the current user's home directory in this file system.
421       * The default implementation returns "/user/$USER/".
422       * 
423       * @return current user's home directory.
424       */
425      public Path getHomeDirectory() {
426        return new Path("/user/"+System.getProperty("user.name")).makeQualified(
427                                                                    getUri(), null);
428      }
429      
430      /**
431       * Return a set of server default configuration values.
432       * 
433       * @return server default configuration values
434       * 
435       * @throws IOException an I/O error occurred
436       */
437      public abstract FsServerDefaults getServerDefaults() throws IOException; 
438    
439      /**
440       * Return the fully-qualified path of path f resolving the path
441       * through any internal symlinks or mount point
442       * @param p path to be resolved
443       * @return fully qualified path 
444       * @throws FileNotFoundException, AccessControlException, IOException
445       *         UnresolvedLinkException if symbolic link on path cannot be resolved
446       *          internally
447       */
448       public Path resolvePath(final Path p) throws FileNotFoundException,
449               UnresolvedLinkException, AccessControlException, IOException {
450         checkPath(p);
451         return getFileStatus(p).getPath(); // default impl is to return the path
452       }
453      
454      /**
455       * The specification of this method matches that of
456       * {@link FileContext#create(Path, EnumSet, Options.CreateOpts...)} except
457       * that the Path f must be fully qualified and the permission is absolute
458       * (i.e. umask has been applied).
459       */
460      public final FSDataOutputStream create(final Path f,
461          final EnumSet<CreateFlag> createFlag, Options.CreateOpts... opts)
462          throws AccessControlException, FileAlreadyExistsException,
463          FileNotFoundException, ParentNotDirectoryException,
464          UnsupportedFileSystemException, UnresolvedLinkException, IOException {
465        checkPath(f);
466        int bufferSize = -1;
467        short replication = -1;
468        long blockSize = -1;
469        int bytesPerChecksum = -1;
470        FsPermission permission = null;
471        Progressable progress = null;
472        Boolean createParent = null;
473     
474        for (CreateOpts iOpt : opts) {
475          if (CreateOpts.BlockSize.class.isInstance(iOpt)) {
476            if (blockSize != -1) {
477              throw new HadoopIllegalArgumentException(
478                  "BlockSize option is set multiple times");
479            }
480            blockSize = ((CreateOpts.BlockSize) iOpt).getValue();
481          } else if (CreateOpts.BufferSize.class.isInstance(iOpt)) {
482            if (bufferSize != -1) {
483              throw new HadoopIllegalArgumentException(
484                  "BufferSize option is set multiple times");
485            }
486            bufferSize = ((CreateOpts.BufferSize) iOpt).getValue();
487          } else if (CreateOpts.ReplicationFactor.class.isInstance(iOpt)) {
488            if (replication != -1) {
489              throw new HadoopIllegalArgumentException(
490                  "ReplicationFactor option is set multiple times");
491            }
492            replication = ((CreateOpts.ReplicationFactor) iOpt).getValue();
493          } else if (CreateOpts.BytesPerChecksum.class.isInstance(iOpt)) {
494            if (bytesPerChecksum != -1) {
495              throw new HadoopIllegalArgumentException(
496                  "BytesPerChecksum option is set multiple times");
497            }
498            bytesPerChecksum = ((CreateOpts.BytesPerChecksum) iOpt).getValue();
499          } else if (CreateOpts.Perms.class.isInstance(iOpt)) {
500            if (permission != null) {
501              throw new HadoopIllegalArgumentException(
502                  "Perms option is set multiple times");
503            }
504            permission = ((CreateOpts.Perms) iOpt).getValue();
505          } else if (CreateOpts.Progress.class.isInstance(iOpt)) {
506            if (progress != null) {
507              throw new HadoopIllegalArgumentException(
508                  "Progress option is set multiple times");
509            }
510            progress = ((CreateOpts.Progress) iOpt).getValue();
511          } else if (CreateOpts.CreateParent.class.isInstance(iOpt)) {
512            if (createParent != null) {
513              throw new HadoopIllegalArgumentException(
514                  "CreateParent option is set multiple times");
515            }
516            createParent = ((CreateOpts.CreateParent) iOpt).getValue();
517          } else {
518            throw new HadoopIllegalArgumentException("Unkown CreateOpts of type " +
519                iOpt.getClass().getName());
520          }
521        }
522        if (permission == null) {
523          throw new HadoopIllegalArgumentException("no permission supplied");
524        }
525    
526    
527        FsServerDefaults ssDef = getServerDefaults();
528        if (ssDef.getBlockSize() % ssDef.getBytesPerChecksum() != 0) {
529          throw new IOException("Internal error: default blockSize is" + 
530              " not a multiple of default bytesPerChecksum ");
531        }
532        
533        if (blockSize == -1) {
534          blockSize = ssDef.getBlockSize();
535        }
536        if (bytesPerChecksum == -1) {
537          bytesPerChecksum = ssDef.getBytesPerChecksum();
538        }
539        if (bufferSize == -1) {
540          bufferSize = ssDef.getFileBufferSize();
541        }
542        if (replication == -1) {
543          replication = ssDef.getReplication();
544        }
545        if (createParent == null) {
546          createParent = false;
547        }
548    
549        if (blockSize % bytesPerChecksum != 0) {
550          throw new HadoopIllegalArgumentException(
551                 "blockSize should be a multiple of checksumsize");
552        }
553    
554        return this.createInternal(f, createFlag, permission, bufferSize,
555          replication, blockSize, progress, bytesPerChecksum, createParent);
556      }
557    
558      /**
559       * The specification of this method matches that of
560       * {@link #create(Path, EnumSet, Options.CreateOpts...)} except that the opts
561       * have been declared explicitly.
562       */
563      public abstract FSDataOutputStream createInternal(Path f,
564          EnumSet<CreateFlag> flag, FsPermission absolutePermission,
565          int bufferSize, short replication, long blockSize, Progressable progress,
566          int bytesPerChecksum, boolean createParent)
567          throws AccessControlException, FileAlreadyExistsException,
568          FileNotFoundException, ParentNotDirectoryException,
569          UnsupportedFileSystemException, UnresolvedLinkException, IOException;
570    
571      /**
572       * The specification of this method matches that of
573       * {@link FileContext#mkdir(Path, FsPermission, boolean)} except that the Path
574       * f must be fully qualified and the permission is absolute (i.e. 
575       * umask has been applied).
576       */
577      public abstract void mkdir(final Path dir, final FsPermission permission,
578          final boolean createParent) throws AccessControlException,
579          FileAlreadyExistsException, FileNotFoundException,
580          UnresolvedLinkException, IOException;
581    
582      /**
583       * The specification of this method matches that of
584       * {@link FileContext#delete(Path, boolean)} except that Path f must be for
585       * this file system.
586       */
587      public abstract boolean delete(final Path f, final boolean recursive)
588          throws AccessControlException, FileNotFoundException,
589          UnresolvedLinkException, IOException;
590    
591      /**
592       * The specification of this method matches that of
593       * {@link FileContext#open(Path)} except that Path f must be for this
594       * file system.
595       */
596      public FSDataInputStream open(final Path f) throws AccessControlException,
597          FileNotFoundException, UnresolvedLinkException, IOException {
598        return open(f, getServerDefaults().getFileBufferSize());
599      }
600    
601      /**
602       * The specification of this method matches that of
603       * {@link FileContext#open(Path, int)} except that Path f must be for this
604       * file system.
605       */
606      public abstract FSDataInputStream open(final Path f, int bufferSize)
607          throws AccessControlException, FileNotFoundException,
608          UnresolvedLinkException, IOException;
609    
610      /**
611       * The specification of this method matches that of
612       * {@link FileContext#setReplication(Path, short)} except that Path f must be
613       * for this file system.
614       */
615      public abstract boolean setReplication(final Path f,
616          final short replication) throws AccessControlException,
617          FileNotFoundException, UnresolvedLinkException, IOException;
618    
619      /**
620       * The specification of this method matches that of
621       * {@link FileContext#rename(Path, Path, Options.Rename...)} except that Path
622       * f must be for this file system.
623       */
624      public final void rename(final Path src, final Path dst,
625          final Options.Rename... options) throws AccessControlException,
626          FileAlreadyExistsException, FileNotFoundException,
627          ParentNotDirectoryException, UnresolvedLinkException, IOException {
628        boolean overwrite = false;
629        if (null != options) {
630          for (Rename option : options) {
631            if (option == Rename.OVERWRITE) {
632              overwrite = true;
633            }
634          }
635        }
636        renameInternal(src, dst, overwrite);
637      }
638      
639      /**
640       * The specification of this method matches that of
641       * {@link FileContext#rename(Path, Path, Options.Rename...)} except that Path
642       * f must be for this file system and NO OVERWRITE is performed.
643       * 
644       * File systems that do not have a built in overwrite need implement only this
645       * method and can take advantage of the default impl of the other
646       * {@link #renameInternal(Path, Path, boolean)}
647       */
648      public abstract void renameInternal(final Path src, final Path dst)
649          throws AccessControlException, FileAlreadyExistsException,
650          FileNotFoundException, ParentNotDirectoryException,
651          UnresolvedLinkException, IOException;
652      
653      /**
654       * The specification of this method matches that of
655       * {@link FileContext#rename(Path, Path, Options.Rename...)} except that Path
656       * f must be for this file system.
657       */
658      public void renameInternal(final Path src, final Path dst,
659          boolean overwrite) throws AccessControlException,
660          FileAlreadyExistsException, FileNotFoundException,
661          ParentNotDirectoryException, UnresolvedLinkException, IOException {
662        // Default implementation deals with overwrite in a non-atomic way
663        final FileStatus srcStatus = getFileLinkStatus(src);
664    
665        FileStatus dstStatus;
666        try {
667          dstStatus = getFileLinkStatus(dst);
668        } catch (IOException e) {
669          dstStatus = null;
670        }
671        if (dstStatus != null) {
672          if (dst.equals(src)) {
673            throw new FileAlreadyExistsException(
674                "The source "+src+" and destination "+dst+" are the same");
675          }
676          if (srcStatus.isSymlink() && dst.equals(srcStatus.getSymlink())) {
677            throw new FileAlreadyExistsException(
678                "Cannot rename symlink "+src+" to its target "+dst);
679          }
680          // It's OK to rename a file to a symlink and vice versa
681          if (srcStatus.isDirectory() != dstStatus.isDirectory()) {
682            throw new IOException("Source " + src + " and destination " + dst
683                + " must both be directories");
684          }
685          if (!overwrite) {
686            throw new FileAlreadyExistsException("Rename destination " + dst
687                + " already exists.");
688          }
689          // Delete the destination that is a file or an empty directory
690          if (dstStatus.isDirectory()) {
691            RemoteIterator<FileStatus> list = listStatusIterator(dst);
692            if (list != null && list.hasNext()) {
693              throw new IOException(
694                  "Rename cannot overwrite non empty destination directory " + dst);
695            }
696          }
697          delete(dst, false);
698        } else {
699          final Path parent = dst.getParent();
700          final FileStatus parentStatus = getFileStatus(parent);
701          if (parentStatus.isFile()) {
702            throw new ParentNotDirectoryException("Rename destination parent "
703                + parent + " is a file.");
704          }
705        }
706        renameInternal(src, dst);
707      }
708      
709      /**
710       * Returns true if the file system supports symlinks, false otherwise.
711       */
712      public boolean supportsSymlinks() {
713        return false;
714      }
715      
716      /**
717       * The specification of this method matches that of  
718       * {@link FileContext#createSymlink(Path, Path, boolean)};
719       */
720      public void createSymlink(final Path target, final Path link,
721          final boolean createParent) throws IOException, UnresolvedLinkException {
722        throw new IOException("File system does not support symlinks");    
723      }
724    
725      /**
726       * The specification of this method matches that of  
727       * {@link FileContext#getLinkTarget(Path)};
728       */
729      public Path getLinkTarget(final Path f) throws IOException {
730        /* We should never get here. Any file system that threw an
731         * UnresolvedLinkException, causing this function to be called,
732         * needs to override this method.
733         */
734        throw new AssertionError();
735      }
736        
737      /**
738       * The specification of this method matches that of
739       * {@link FileContext#setPermission(Path, FsPermission)} except that Path f
740       * must be for this file system.
741       */
742      public abstract void setPermission(final Path f,
743          final FsPermission permission) throws AccessControlException,
744          FileNotFoundException, UnresolvedLinkException, IOException;
745    
746      /**
747       * The specification of this method matches that of
748       * {@link FileContext#setOwner(Path, String, String)} except that Path f must
749       * be for this file system.
750       */
751      public abstract void setOwner(final Path f, final String username,
752          final String groupname) throws AccessControlException,
753          FileNotFoundException, UnresolvedLinkException, IOException;
754    
755      /**
756       * The specification of this method matches that of
757       * {@link FileContext#setTimes(Path, long, long)} except that Path f must be
758       * for this file system.
759       */
760      public abstract void setTimes(final Path f, final long mtime,
761        final long atime) throws AccessControlException, FileNotFoundException,
762          UnresolvedLinkException, IOException;
763    
764      /**
765       * The specification of this method matches that of
766       * {@link FileContext#getFileChecksum(Path)} except that Path f must be for
767       * this file system.
768       */
769      public abstract FileChecksum getFileChecksum(final Path f)
770          throws AccessControlException, FileNotFoundException,
771          UnresolvedLinkException, IOException;
772      
773      /**
774       * The specification of this method matches that of
775       * {@link FileContext#getFileStatus(Path)} 
776       * except that an UnresolvedLinkException may be thrown if a symlink is 
777       * encountered in the path.
778       */
779      public abstract FileStatus getFileStatus(final Path f)
780          throws AccessControlException, FileNotFoundException,
781          UnresolvedLinkException, IOException;
782    
783      /**
784       * The specification of this method matches that of
785       * {@link FileContext#getFileLinkStatus(Path)}
786       * except that an UnresolvedLinkException may be thrown if a symlink is  
787       * encountered in the path leading up to the final path component.
788       * If the file system does not support symlinks then the behavior is
789       * equivalent to {@link AbstractFileSystem#getFileStatus(Path)}.
790       */
791      public FileStatus getFileLinkStatus(final Path f)
792          throws AccessControlException, FileNotFoundException,
793          UnsupportedFileSystemException, IOException {
794        return getFileStatus(f);
795      }
796    
797      /**
798       * The specification of this method matches that of
799       * {@link FileContext#getFileBlockLocations(Path, long, long)} except that
800       * Path f must be for this file system.
801       */
802      public abstract BlockLocation[] getFileBlockLocations(final Path f,
803          final long start, final long len) throws AccessControlException,
804          FileNotFoundException, UnresolvedLinkException, IOException;
805    
806      /**
807       * The specification of this method matches that of
808       * {@link FileContext#getFsStatus(Path)} except that Path f must be for this
809       * file system.
810       */
811      public FsStatus getFsStatus(final Path f) throws AccessControlException,
812          FileNotFoundException, UnresolvedLinkException, IOException {
813        // default impl gets FsStatus of root
814        return getFsStatus();
815      }
816      
817      /**
818       * The specification of this method matches that of
819       * {@link FileContext#getFsStatus(Path)}.
820       */
821      public abstract FsStatus getFsStatus() throws AccessControlException,
822          FileNotFoundException, IOException;
823    
824      /**
825       * The specification of this method matches that of
826       * {@link FileContext#listStatus(Path)} except that Path f must be for this
827       * file system.
828       */
829      public RemoteIterator<FileStatus> listStatusIterator(final Path f)
830          throws AccessControlException, FileNotFoundException,
831          UnresolvedLinkException, IOException {
832        return new RemoteIterator<FileStatus>() {
833          private int i = 0;
834          private FileStatus[] statusList = listStatus(f);
835          
836          @Override
837          public boolean hasNext() {
838            return i < statusList.length;
839          }
840          
841          @Override
842          public FileStatus next() {
843            if (!hasNext()) {
844              throw new NoSuchElementException();
845            }
846            return statusList[i++];
847          }
848        };
849      }
850    
851      /**
852       * The specification of this method matches that of
853       * {@link FileContext#listLocatedStatus(Path)} except that Path f 
854       * must be for this file system.
855       */
856      public RemoteIterator<LocatedFileStatus> listLocatedStatus(final Path f)
857          throws AccessControlException, FileNotFoundException,
858          UnresolvedLinkException, IOException {
859        return new RemoteIterator<LocatedFileStatus>() {
860          private RemoteIterator<FileStatus> itor = listStatusIterator(f);
861          
862          @Override
863          public boolean hasNext() throws IOException {
864            return itor.hasNext();
865          }
866          
867          @Override
868          public LocatedFileStatus next() throws IOException {
869            if (!hasNext()) {
870              throw new NoSuchElementException("No more entry in " + f);
871            }
872            FileStatus result = itor.next();
873            BlockLocation[] locs = null;
874            if (result.isFile()) {
875              locs = getFileBlockLocations(
876                  result.getPath(), 0, result.getLen());
877            }
878            return new LocatedFileStatus(result, locs);
879          }
880        };
881      }
882    
883      /**
884       * The specification of this method matches that of
885       * {@link FileContext.Util#listStatus(Path)} except that Path f must be 
886       * for this file system.
887       */
888      public abstract FileStatus[] listStatus(final Path f)
889          throws AccessControlException, FileNotFoundException,
890          UnresolvedLinkException, IOException;
891    
892      /**
893       * @return an iterator over the corrupt files under the given path
894       * (may contain duplicates if a file has more than one corrupt block)
895       * @throws IOException
896       */
897      public RemoteIterator<Path> listCorruptFileBlocks(Path path)
898        throws IOException {
899        throw new UnsupportedOperationException(getClass().getCanonicalName() +
900                                                " does not support" +
901                                                " listCorruptFileBlocks");
902      }
903    
904      /**
905       * The specification of this method matches that of
906       * {@link FileContext#setVerifyChecksum(boolean, Path)} except that Path f
907       * must be for this file system.
908       */
909      public abstract void setVerifyChecksum(final boolean verifyChecksum)
910          throws AccessControlException, IOException;
911      
912      /**
913       * Get a canonical name for this file system.
914       * @return a URI string that uniquely identifies this file system
915       */
916      public String getCanonicalServiceName() {
917        return SecurityUtil.buildDTServiceName(getUri(), getUriDefaultPort());
918      }
919      
920      /**
921       * Get one or more delegation tokens associated with the filesystem. Normally
922       * a file system returns a single delegation token. A file system that manages
923       * multiple file systems underneath, could return set of delegation tokens for
924       * all the file systems it manages
925       * 
926       * @param renewer the account name that is allowed to renew the token.
927       * @return List of delegation tokens.
928       *   If delegation tokens not supported then return a list of size zero.
929       * @throws IOException
930       */
931      @InterfaceAudience.LimitedPrivate( { "HDFS", "MapReduce" })
932      public List<Token<?>> getDelegationTokens(String renewer) throws IOException {
933        return new ArrayList<Token<?>>(0);
934      }
935      
936      @Override //Object
937      public int hashCode() {
938        return myUri.hashCode();
939      }
940      
941      @Override //Object
942      public boolean equals(Object other) {
943        if (other == null || !(other instanceof AbstractFileSystem)) {
944          return false;
945        }
946        return myUri.equals(((AbstractFileSystem) other).myUri);
947      }
948    }