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