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    
019    package org.apache.hadoop.hdfs.web;
020    
021    import java.io.BufferedOutputStream;
022    import java.io.FileNotFoundException;
023    import java.io.IOException;
024    import java.io.InputStream;
025    import java.io.InputStreamReader;
026    import java.net.HttpURLConnection;
027    import java.net.InetSocketAddress;
028    import java.net.MalformedURLException;
029    import java.net.URI;
030    import java.net.URISyntaxException;
031    import java.net.URL;
032    import java.security.PrivilegedExceptionAction;
033    import java.util.Collection;
034    import java.util.List;
035    import java.util.Map;
036    import java.util.StringTokenizer;
037    
038    import javax.ws.rs.core.MediaType;
039    
040    import org.apache.commons.logging.Log;
041    import org.apache.commons.logging.LogFactory;
042    import org.apache.hadoop.conf.Configuration;
043    import org.apache.hadoop.fs.BlockLocation;
044    import org.apache.hadoop.fs.ContentSummary;
045    import org.apache.hadoop.fs.DelegationTokenRenewer;
046    import org.apache.hadoop.fs.FSDataInputStream;
047    import org.apache.hadoop.fs.FSDataOutputStream;
048    import org.apache.hadoop.fs.FileStatus;
049    import org.apache.hadoop.fs.FileSystem;
050    import org.apache.hadoop.fs.MD5MD5CRC32FileChecksum;
051    import org.apache.hadoop.fs.Options;
052    import org.apache.hadoop.fs.Path;
053    import org.apache.hadoop.fs.permission.FsPermission;
054    import org.apache.hadoop.hdfs.ByteRangeInputStream;
055    import org.apache.hadoop.hdfs.DFSConfigKeys;
056    import org.apache.hadoop.hdfs.DFSUtil;
057    import org.apache.hadoop.hdfs.protocol.HdfsFileStatus;
058    import org.apache.hadoop.hdfs.security.token.delegation.DelegationTokenIdentifier;
059    import org.apache.hadoop.hdfs.security.token.delegation.DelegationTokenSelector;
060    import org.apache.hadoop.hdfs.server.namenode.SafeModeException;
061    import org.apache.hadoop.hdfs.web.resources.AccessTimeParam;
062    import org.apache.hadoop.hdfs.web.resources.BlockSizeParam;
063    import org.apache.hadoop.hdfs.web.resources.BufferSizeParam;
064    import org.apache.hadoop.hdfs.web.resources.ConcatSourcesParam;
065    import org.apache.hadoop.hdfs.web.resources.CreateParentParam;
066    import org.apache.hadoop.hdfs.web.resources.DelegationParam;
067    import org.apache.hadoop.hdfs.web.resources.DeleteOpParam;
068    import org.apache.hadoop.hdfs.web.resources.DestinationParam;
069    import org.apache.hadoop.hdfs.web.resources.DoAsParam;
070    import org.apache.hadoop.hdfs.web.resources.GetOpParam;
071    import org.apache.hadoop.hdfs.web.resources.GroupParam;
072    import org.apache.hadoop.hdfs.web.resources.HttpOpParam;
073    import org.apache.hadoop.hdfs.web.resources.LengthParam;
074    import org.apache.hadoop.hdfs.web.resources.ModificationTimeParam;
075    import org.apache.hadoop.hdfs.web.resources.OffsetParam;
076    import org.apache.hadoop.hdfs.web.resources.OverwriteParam;
077    import org.apache.hadoop.hdfs.web.resources.OwnerParam;
078    import org.apache.hadoop.hdfs.web.resources.Param;
079    import org.apache.hadoop.hdfs.web.resources.PermissionParam;
080    import org.apache.hadoop.hdfs.web.resources.PostOpParam;
081    import org.apache.hadoop.hdfs.web.resources.PutOpParam;
082    import org.apache.hadoop.hdfs.web.resources.RecursiveParam;
083    import org.apache.hadoop.hdfs.web.resources.RenameOptionSetParam;
084    import org.apache.hadoop.hdfs.web.resources.RenewerParam;
085    import org.apache.hadoop.hdfs.web.resources.ReplicationParam;
086    import org.apache.hadoop.hdfs.web.resources.TokenArgumentParam;
087    import org.apache.hadoop.hdfs.web.resources.UserParam;
088    import org.apache.hadoop.io.Text;
089    import org.apache.hadoop.io.retry.RetryPolicy;
090    import org.apache.hadoop.io.retry.RetryUtils;
091    import org.apache.hadoop.ipc.RemoteException;
092    import org.apache.hadoop.net.NetUtils;
093    import org.apache.hadoop.security.SecurityUtil;
094    import org.apache.hadoop.security.UserGroupInformation;
095    import org.apache.hadoop.security.authentication.client.AuthenticatedURL;
096    import org.apache.hadoop.security.authentication.client.AuthenticationException;
097    import org.apache.hadoop.security.authorize.AuthorizationException;
098    import org.apache.hadoop.security.authentication.client.ConnectionConfigurator;
099    import org.apache.hadoop.security.token.SecretManager.InvalidToken;
100    import org.apache.hadoop.security.token.Token;
101    import org.apache.hadoop.security.token.TokenIdentifier;
102    import org.apache.hadoop.security.token.TokenRenewer;
103    import org.apache.hadoop.security.token.delegation.AbstractDelegationTokenSelector;
104    import org.apache.hadoop.util.Progressable;
105    import org.mortbay.util.ajax.JSON;
106    
107    import com.google.common.annotations.VisibleForTesting;
108    import com.google.common.base.Charsets;
109    import com.google.common.collect.Lists;
110    
111    /** A FileSystem for HDFS over the web. */
112    public class WebHdfsFileSystem extends FileSystem
113        implements DelegationTokenRenewer.Renewable {
114      public static final Log LOG = LogFactory.getLog(WebHdfsFileSystem.class);
115      /** File System URI: {SCHEME}://namenode:port/path/to/file */
116      public static final String SCHEME = "webhdfs";
117      /** WebHdfs version. */
118      public static final int VERSION = 1;
119      /** Http URI: http://namenode:port/{PATH_PREFIX}/path/to/file */
120      public static final String PATH_PREFIX = "/" + SCHEME + "/v" + VERSION;
121    
122      /** SPNEGO authenticator */
123      private static final KerberosUgiAuthenticator AUTH = new KerberosUgiAuthenticator();
124      /** Configures connections for AuthenticatedURL */
125      private static final ConnectionConfigurator CONN_CONFIGURATOR =
126        new ConnectionConfigurator() {
127          @Override
128          public HttpURLConnection configure(HttpURLConnection conn)
129              throws IOException {
130            URLUtils.setTimeouts(conn);
131            return conn;
132          }
133        };
134      /** Delegation token kind */
135      public static final Text TOKEN_KIND = new Text("WEBHDFS delegation");
136      /** Token selector */
137      public static final WebHdfsDelegationTokenSelector DT_SELECTOR
138          = new WebHdfsDelegationTokenSelector();
139    
140      private DelegationTokenRenewer dtRenewer = null;
141      @VisibleForTesting
142      DelegationTokenRenewer.RenewAction<?> action;
143    
144      @VisibleForTesting
145      protected synchronized void addRenewAction(final WebHdfsFileSystem webhdfs) {
146        if (dtRenewer == null) {
147          dtRenewer = DelegationTokenRenewer.getInstance();
148        }
149    
150        action = dtRenewer.addRenewAction(webhdfs);
151      }
152    
153      /** Is WebHDFS enabled in conf? */
154      public static boolean isEnabled(final Configuration conf, final Log log) {
155        final boolean b = conf.getBoolean(DFSConfigKeys.DFS_WEBHDFS_ENABLED_KEY,
156            DFSConfigKeys.DFS_WEBHDFS_ENABLED_DEFAULT);
157        log.info(DFSConfigKeys.DFS_WEBHDFS_ENABLED_KEY + " = " + b);
158        return b;
159      }
160    
161      private UserGroupInformation ugi;
162      private InetSocketAddress nnAddr;
163      private URI uri;
164      private boolean hasInitedToken;
165      private Token<?> delegationToken;
166      private RetryPolicy retryPolicy = null;
167      private Path workingDir;
168    
169      /**
170       * Return the protocol scheme for the FileSystem.
171       * <p/>
172       *
173       * @return <code>webhdfs</code>
174       */
175      @Override
176      public String getScheme() {
177        return "webhdfs";
178      }
179    
180      @Override
181      public synchronized void initialize(URI uri, Configuration conf
182          ) throws IOException {
183        super.initialize(uri, conf);
184        setConf(conf);
185        ugi = UserGroupInformation.getCurrentUser();
186        try {
187          this.uri = new URI(uri.getScheme(), uri.getAuthority(), null, null, null);
188        } catch (URISyntaxException e) {
189          throw new IllegalArgumentException(e);
190        }
191        this.nnAddr = NetUtils.createSocketAddr(uri.getAuthority(), getDefaultPort());
192        this.retryPolicy = 
193            RetryUtils.getDefaultRetryPolicy(
194                conf, 
195                DFSConfigKeys.DFS_CLIENT_RETRY_POLICY_ENABLED_KEY, 
196                DFSConfigKeys.DFS_CLIENT_RETRY_POLICY_ENABLED_DEFAULT, 
197                DFSConfigKeys.DFS_CLIENT_RETRY_POLICY_SPEC_KEY,
198                DFSConfigKeys.DFS_CLIENT_RETRY_POLICY_SPEC_DEFAULT,
199                SafeModeException.class);
200        this.workingDir = getHomeDirectory();
201    
202        if (UserGroupInformation.isSecurityEnabled()) {
203          initDelegationToken();
204        }
205      }
206    
207      protected void initDelegationToken() throws IOException {
208        // look for webhdfs token, then try hdfs
209        Token<?> token = selectDelegationToken(ugi);
210        if (token != null) {
211          LOG.debug("Found existing DT for " + token.getService());        
212          setDelegationToken(token);
213          hasInitedToken = true;
214        }
215      }
216    
217      protected synchronized Token<?> getDelegationToken() throws IOException {
218        // we haven't inited yet, or we used to have a token but it expired
219        if (!hasInitedToken || (action != null && !action.isValid())) {
220          //since we don't already have a token, go get one
221          Token<?> token = getDelegationToken(null);
222          // security might be disabled
223          if (token != null) {
224            setDelegationToken(token);
225            addRenewAction(this);
226            LOG.debug("Created new DT for " + token.getService());
227          }
228          hasInitedToken = true;
229        }
230        return delegationToken;
231      }
232    
233      protected Token<DelegationTokenIdentifier> selectDelegationToken(
234          UserGroupInformation ugi) {
235        return DT_SELECTOR.selectToken(getCanonicalUri(), ugi.getTokens(), getConf());
236      }
237    
238      @Override
239      protected int getDefaultPort() {
240        return getConf().getInt(DFSConfigKeys.DFS_NAMENODE_HTTP_PORT_KEY,
241            DFSConfigKeys.DFS_NAMENODE_HTTP_PORT_DEFAULT);
242      }
243    
244      @Override
245      public URI getUri() {
246        return this.uri;
247      }
248      
249      @Override
250      protected URI canonicalizeUri(URI uri) {
251        return NetUtils.getCanonicalUri(uri, getDefaultPort());
252      }
253    
254      /** @return the home directory. */
255      public static String getHomeDirectoryString(final UserGroupInformation ugi) {
256        return "/user/" + ugi.getShortUserName();
257      }
258    
259      @Override
260      public Path getHomeDirectory() {
261        return makeQualified(new Path(getHomeDirectoryString(ugi)));
262      }
263    
264      @Override
265      public synchronized Path getWorkingDirectory() {
266        return workingDir;
267      }
268    
269      @Override
270      public synchronized void setWorkingDirectory(final Path dir) {
271        String result = makeAbsolute(dir).toUri().getPath();
272        if (!DFSUtil.isValidName(result)) {
273          throw new IllegalArgumentException("Invalid DFS directory name " + 
274                                             result);
275        }
276        workingDir = makeAbsolute(dir);
277      }
278    
279      private Path makeAbsolute(Path f) {
280        return f.isAbsolute()? f: new Path(workingDir, f);
281      }
282    
283      static Map<?, ?> jsonParse(final HttpURLConnection c, final boolean useErrorStream
284          ) throws IOException {
285        if (c.getContentLength() == 0) {
286          return null;
287        }
288        final InputStream in = useErrorStream? c.getErrorStream(): c.getInputStream();
289        if (in == null) {
290          throw new IOException("The " + (useErrorStream? "error": "input") + " stream is null.");
291        }
292        final String contentType = c.getContentType();
293        if (contentType != null) {
294          final MediaType parsed = MediaType.valueOf(contentType);
295          if (!MediaType.APPLICATION_JSON_TYPE.isCompatible(parsed)) {
296            throw new IOException("Content-Type \"" + contentType
297                + "\" is incompatible with \"" + MediaType.APPLICATION_JSON
298                + "\" (parsed=\"" + parsed + "\")");
299          }
300        }
301        return (Map<?, ?>)JSON.parse(new InputStreamReader(in, Charsets.UTF_8));
302      }
303    
304      private static Map<?, ?> validateResponse(final HttpOpParam.Op op,
305          final HttpURLConnection conn, boolean unwrapException) throws IOException {
306        final int code = conn.getResponseCode();
307        if (code != op.getExpectedHttpResponseCode()) {
308          final Map<?, ?> m;
309          try {
310            m = jsonParse(conn, true);
311          } catch(Exception e) {
312            throw new IOException("Unexpected HTTP response: code=" + code + " != "
313                + op.getExpectedHttpResponseCode() + ", " + op.toQueryString()
314                + ", message=" + conn.getResponseMessage(), e);
315          }
316    
317          if (m == null) {
318            throw new IOException("Unexpected HTTP response: code=" + code + " != "
319                + op.getExpectedHttpResponseCode() + ", " + op.toQueryString()
320                + ", message=" + conn.getResponseMessage());
321          } else if (m.get(RemoteException.class.getSimpleName()) == null) {
322            return m;
323          }
324    
325          final RemoteException re = JsonUtil.toRemoteException(m);
326          throw unwrapException? toIOException(re): re;
327        }
328        return null;
329      }
330    
331      /**
332       * Covert an exception to an IOException.
333       * 
334       * For a non-IOException, wrap it with IOException.
335       * For a RemoteException, unwrap it.
336       * For an IOException which is not a RemoteException, return it. 
337       */
338      private static IOException toIOException(Exception e) {
339        if (!(e instanceof IOException)) {
340          return new IOException(e);
341        }
342    
343        final IOException ioe = (IOException)e;
344        if (!(ioe instanceof RemoteException)) {
345          return ioe;
346        }
347    
348        return ((RemoteException)ioe).unwrapRemoteException();
349      }
350    
351      /**
352       * Return a URL pointing to given path on the namenode.
353       *
354       * @param path to obtain the URL for
355       * @param query string to append to the path
356       * @return namenode URL referring to the given path
357       * @throws IOException on error constructing the URL
358       */
359      private URL getNamenodeURL(String path, String query) throws IOException {
360        final URL url = new URL("http", nnAddr.getHostName(),
361              nnAddr.getPort(), path + '?' + query);
362        if (LOG.isTraceEnabled()) {
363          LOG.trace("url=" + url);
364        }
365        return url;
366      }
367      
368      Param<?,?>[] getAuthParameters(final HttpOpParam.Op op) throws IOException {
369        List<Param<?,?>> authParams = Lists.newArrayList();    
370        // Skip adding delegation token for token operations because these
371        // operations require authentication.
372        Token<?> token = null;
373        if (UserGroupInformation.isSecurityEnabled() && !op.getRequireAuth()) {
374          token = getDelegationToken();
375        }
376        if (token != null) {
377          authParams.add(new DelegationParam(token.encodeToUrlString()));
378        } else {
379          UserGroupInformation userUgi = ugi;
380          UserGroupInformation realUgi = userUgi.getRealUser();
381          if (realUgi != null) { // proxy user
382            authParams.add(new DoAsParam(userUgi.getShortUserName()));
383            userUgi = realUgi;
384          }
385          authParams.add(new UserParam(userUgi.getShortUserName()));
386        }
387        return authParams.toArray(new Param<?,?>[0]);
388      }
389    
390      URL toUrl(final HttpOpParam.Op op, final Path fspath,
391          final Param<?,?>... parameters) throws IOException {
392        //initialize URI path and query
393        final String path = PATH_PREFIX
394            + (fspath == null? "/": makeQualified(fspath).toUri().getRawPath());
395        final String query = op.toQueryString()
396            + Param.toSortedString("&", getAuthParameters(op))
397            + Param.toSortedString("&", parameters);
398        final URL url = getNamenodeURL(path, query);
399        if (LOG.isTraceEnabled()) {
400          LOG.trace("url=" + url);
401        }
402        return url;
403      }
404    
405      /**
406       * Run a http operation.
407       * Connect to the http server, validate response, and obtain the JSON output.
408       * 
409       * @param op http operation
410       * @param fspath file system path
411       * @param parameters parameters for the operation
412       * @return a JSON object, e.g. Object[], Map<?, ?>, etc.
413       * @throws IOException
414       */
415      private Map<?, ?> run(final HttpOpParam.Op op, final Path fspath,
416          final Param<?,?>... parameters) throws IOException {
417        return new Runner(op, fspath, parameters).run().json;
418      }
419    
420      /**
421       * This class is for initialing a HTTP connection, connecting to server,
422       * obtaining a response, and also handling retry on failures.
423       */
424      class Runner {
425        private final HttpOpParam.Op op;
426        private final URL url;
427        private final boolean redirected;
428    
429        private boolean checkRetry;
430        private HttpURLConnection conn = null;
431        private Map<?, ?> json = null;
432    
433        Runner(final HttpOpParam.Op op, final URL url, final boolean redirected) {
434          this.op = op;
435          this.url = url;
436          this.redirected = redirected;
437        }
438    
439        Runner(final HttpOpParam.Op op, final Path fspath,
440            final Param<?,?>... parameters) throws IOException {
441          this(op, toUrl(op, fspath, parameters), false);
442        }
443    
444        Runner(final HttpOpParam.Op op, final HttpURLConnection conn) {
445          this(op, null, false);
446          this.conn = conn;
447        }
448    
449        private HttpURLConnection getHttpUrlConnection(final URL url)
450            throws IOException, AuthenticationException {
451          UserGroupInformation connectUgi = ugi.getRealUser();
452          if (connectUgi == null) {
453            connectUgi = ugi;
454          }
455          try {
456            return connectUgi.doAs(
457                new PrivilegedExceptionAction<HttpURLConnection>() {
458                  @Override
459                  public HttpURLConnection run() throws IOException {
460                    return openHttpUrlConnection(url);
461                  }
462                });
463          } catch (IOException ioe) {
464            Throwable cause = ioe.getCause();
465            if (cause != null && cause instanceof AuthenticationException) {
466              throw (AuthenticationException)cause;
467            }
468            throw ioe;
469          } catch (InterruptedException e) {
470            throw new IOException(e);
471          }
472        }
473        
474        private HttpURLConnection openHttpUrlConnection(final URL url)
475            throws IOException {
476          final HttpURLConnection conn;
477          try {
478            if (op.getRequireAuth()) {
479              LOG.debug("open AuthenticatedURL connection");
480              UserGroupInformation.getCurrentUser().checkTGTAndReloginFromKeytab();
481              final AuthenticatedURL.Token authToken = new AuthenticatedURL.Token();
482              conn = new AuthenticatedURL(AUTH, CONN_CONFIGURATOR).openConnection(
483                url, authToken);
484              URLUtils.setTimeouts(conn);
485            } else {
486              LOG.debug("open URL connection");
487              conn = (HttpURLConnection)URLUtils.openConnection(url);
488            }
489          } catch (AuthenticationException e) {
490            throw new IOException(e);
491          }
492          return conn;
493        }
494      
495        private void init() throws IOException {
496          checkRetry = !redirected;
497          try {
498            conn = getHttpUrlConnection(url);
499          } catch(AuthenticationException ae) {
500            checkRetry = false;
501            throw new IOException("Authentication failed, url=" + url, ae);
502          }
503        }
504        
505        private void connect() throws IOException {
506          connect(op.getDoOutput());
507        }
508    
509        private void connect(boolean doOutput) throws IOException {
510          conn.setRequestMethod(op.getType().toString());
511          conn.setDoOutput(doOutput);
512          conn.setInstanceFollowRedirects(false);
513          conn.connect();
514        }
515    
516        private void disconnect() {
517          if (conn != null) {
518            conn.disconnect();
519            conn = null;
520          }
521        }
522    
523        Runner run() throws IOException {
524          for(int retry = 0; ; retry++) {
525            try {
526              init();
527              if (op.getDoOutput()) {
528                twoStepWrite();
529              } else {
530                getResponse(op != GetOpParam.Op.OPEN);
531              }
532              return this;
533            } catch(IOException ioe) {
534              shouldRetry(ioe, retry);
535            }
536          }
537        }
538    
539        private void shouldRetry(final IOException ioe, final int retry
540            ) throws IOException {
541          if (checkRetry) {
542            try {
543              final RetryPolicy.RetryAction a = retryPolicy.shouldRetry(
544                  ioe, retry, 0, true);
545              if (a.action == RetryPolicy.RetryAction.RetryDecision.RETRY) {
546                LOG.info("Retrying connect to namenode: " + nnAddr
547                    + ". Already tried " + retry + " time(s); retry policy is "
548                    + retryPolicy + ", delay " + a.delayMillis + "ms.");      
549                Thread.sleep(a.delayMillis);
550                return;
551              }
552            } catch(Exception e) {
553              LOG.warn("Original exception is ", ioe);
554              throw toIOException(e);
555            }
556          }
557          throw toIOException(ioe);
558        }
559    
560        /**
561         * Two-step Create/Append:
562         * Step 1) Submit a Http request with neither auto-redirect nor data. 
563         * Step 2) Submit another Http request with the URL from the Location header with data.
564         * 
565         * The reason of having two-step create/append is for preventing clients to
566         * send out the data before the redirect. This issue is addressed by the
567         * "Expect: 100-continue" header in HTTP/1.1; see RFC 2616, Section 8.2.3.
568         * Unfortunately, there are software library bugs (e.g. Jetty 6 http server
569         * and Java 6 http client), which do not correctly implement "Expect:
570         * 100-continue". The two-step create/append is a temporary workaround for
571         * the software library bugs.
572         */
573        HttpURLConnection twoStepWrite() throws IOException {
574          //Step 1) Submit a Http request with neither auto-redirect nor data. 
575          connect(false);
576          validateResponse(HttpOpParam.TemporaryRedirectOp.valueOf(op), conn, false);
577          final String redirect = conn.getHeaderField("Location");
578          disconnect();
579          checkRetry = false;
580          
581          //Step 2) Submit another Http request with the URL from the Location header with data.
582          conn = (HttpURLConnection)URLUtils.openConnection(new URL(redirect));
583          conn.setRequestProperty("Content-Type", MediaType.APPLICATION_OCTET_STREAM);
584          conn.setChunkedStreamingMode(32 << 10); //32kB-chunk
585          connect();
586          return conn;
587        }
588    
589        FSDataOutputStream write(final int bufferSize) throws IOException {
590          return WebHdfsFileSystem.this.write(op, conn, bufferSize);
591        }
592    
593        void getResponse(boolean getJsonAndDisconnect) throws IOException {
594          try {
595            connect();
596            final int code = conn.getResponseCode();
597            if (!redirected && op.getRedirect()
598                && code != op.getExpectedHttpResponseCode()) {
599              final String redirect = conn.getHeaderField("Location");
600              json = validateResponse(HttpOpParam.TemporaryRedirectOp.valueOf(op),
601                  conn, false);
602              disconnect();
603      
604              checkRetry = false;
605              conn = (HttpURLConnection)URLUtils.openConnection(new URL(redirect));
606              connect();
607            }
608    
609            json = validateResponse(op, conn, false);
610            if (json == null && getJsonAndDisconnect) {
611              json = jsonParse(conn, false);
612            }
613          } finally {
614            if (getJsonAndDisconnect) {
615              disconnect();
616            }
617          }
618        }
619      }
620    
621      private FsPermission applyUMask(FsPermission permission) {
622        if (permission == null) {
623          permission = FsPermission.getDefault();
624        }
625        return permission.applyUMask(FsPermission.getUMask(getConf()));
626      }
627    
628      private HdfsFileStatus getHdfsFileStatus(Path f) throws IOException {
629        final HttpOpParam.Op op = GetOpParam.Op.GETFILESTATUS;
630        final Map<?, ?> json = run(op, f);
631        final HdfsFileStatus status = JsonUtil.toFileStatus(json, true);
632        if (status == null) {
633          throw new FileNotFoundException("File does not exist: " + f);
634        }
635        return status;
636      }
637    
638      @Override
639      public FileStatus getFileStatus(Path f) throws IOException {
640        statistics.incrementReadOps(1);
641        return makeQualified(getHdfsFileStatus(f), f);
642      }
643    
644      private FileStatus makeQualified(HdfsFileStatus f, Path parent) {
645        return new FileStatus(f.getLen(), f.isDir(), f.getReplication(),
646            f.getBlockSize(), f.getModificationTime(), f.getAccessTime(),
647            f.getPermission(), f.getOwner(), f.getGroup(),
648            f.isSymlink() ? new Path(f.getSymlink()) : null,
649            f.getFullPath(parent).makeQualified(getUri(), getWorkingDirectory()));
650      }
651    
652      @Override
653      public boolean mkdirs(Path f, FsPermission permission) throws IOException {
654        statistics.incrementWriteOps(1);
655        final HttpOpParam.Op op = PutOpParam.Op.MKDIRS;
656        final Map<?, ?> json = run(op, f,
657            new PermissionParam(applyUMask(permission)));
658        return (Boolean)json.get("boolean");
659      }
660    
661      /**
662       * Create a symlink pointing to the destination path.
663       * @see org.apache.hadoop.fs.Hdfs#createSymlink(Path, Path, boolean) 
664       */
665      public void createSymlink(Path destination, Path f, boolean createParent
666          ) throws IOException {
667        statistics.incrementWriteOps(1);
668        final HttpOpParam.Op op = PutOpParam.Op.CREATESYMLINK;
669        run(op, f, new DestinationParam(makeQualified(destination).toUri().getPath()),
670            new CreateParentParam(createParent));
671      }
672    
673      @Override
674      public boolean rename(final Path src, final Path dst) throws IOException {
675        statistics.incrementWriteOps(1);
676        final HttpOpParam.Op op = PutOpParam.Op.RENAME;
677        final Map<?, ?> json = run(op, src,
678            new DestinationParam(makeQualified(dst).toUri().getPath()));
679        return (Boolean)json.get("boolean");
680      }
681    
682      @SuppressWarnings("deprecation")
683      @Override
684      public void rename(final Path src, final Path dst,
685          final Options.Rename... options) throws IOException {
686        statistics.incrementWriteOps(1);
687        final HttpOpParam.Op op = PutOpParam.Op.RENAME;
688        run(op, src, new DestinationParam(makeQualified(dst).toUri().getPath()),
689            new RenameOptionSetParam(options));
690      }
691    
692      @Override
693      public void setOwner(final Path p, final String owner, final String group
694          ) throws IOException {
695        if (owner == null && group == null) {
696          throw new IOException("owner == null && group == null");
697        }
698    
699        statistics.incrementWriteOps(1);
700        final HttpOpParam.Op op = PutOpParam.Op.SETOWNER;
701        run(op, p, new OwnerParam(owner), new GroupParam(group));
702      }
703    
704      @Override
705      public void setPermission(final Path p, final FsPermission permission
706          ) throws IOException {
707        statistics.incrementWriteOps(1);
708        final HttpOpParam.Op op = PutOpParam.Op.SETPERMISSION;
709        run(op, p, new PermissionParam(permission));
710      }
711    
712      @Override
713      public boolean setReplication(final Path p, final short replication
714         ) throws IOException {
715        statistics.incrementWriteOps(1);
716        final HttpOpParam.Op op = PutOpParam.Op.SETREPLICATION;
717        final Map<?, ?> json = run(op, p, new ReplicationParam(replication));
718        return (Boolean)json.get("boolean");
719      }
720    
721      @Override
722      public void setTimes(final Path p, final long mtime, final long atime
723          ) throws IOException {
724        statistics.incrementWriteOps(1);
725        final HttpOpParam.Op op = PutOpParam.Op.SETTIMES;
726        run(op, p, new ModificationTimeParam(mtime), new AccessTimeParam(atime));
727      }
728    
729      @Override
730      public long getDefaultBlockSize() {
731        return getConf().getLongBytes(DFSConfigKeys.DFS_BLOCK_SIZE_KEY,
732            DFSConfigKeys.DFS_BLOCK_SIZE_DEFAULT);
733      }
734    
735      @Override
736      public short getDefaultReplication() {
737        return (short)getConf().getInt(DFSConfigKeys.DFS_REPLICATION_KEY,
738            DFSConfigKeys.DFS_REPLICATION_DEFAULT);
739      }
740    
741      FSDataOutputStream write(final HttpOpParam.Op op,
742          final HttpURLConnection conn, final int bufferSize) throws IOException {
743        return new FSDataOutputStream(new BufferedOutputStream(
744            conn.getOutputStream(), bufferSize), statistics) {
745          @Override
746          public void close() throws IOException {
747            try {
748              super.close();
749            } finally {
750              try {
751                validateResponse(op, conn, true);
752              } finally {
753                conn.disconnect();
754              }
755            }
756          }
757        };
758      }
759    
760      @Override
761      public void concat(final Path trg, final Path [] srcs) throws IOException {
762        statistics.incrementWriteOps(1);
763        final HttpOpParam.Op op = PostOpParam.Op.CONCAT;
764    
765        ConcatSourcesParam param = new ConcatSourcesParam(srcs);
766        run(op, trg, param);
767      }
768    
769      @Override
770      public FSDataOutputStream create(final Path f, final FsPermission permission,
771          final boolean overwrite, final int bufferSize, final short replication,
772          final long blockSize, final Progressable progress) throws IOException {
773        statistics.incrementWriteOps(1);
774    
775        final HttpOpParam.Op op = PutOpParam.Op.CREATE;
776        return new Runner(op, f, 
777            new PermissionParam(applyUMask(permission)),
778            new OverwriteParam(overwrite),
779            new BufferSizeParam(bufferSize),
780            new ReplicationParam(replication),
781            new BlockSizeParam(blockSize))
782          .run()
783          .write(bufferSize);
784      }
785    
786      @Override
787      public FSDataOutputStream append(final Path f, final int bufferSize,
788          final Progressable progress) throws IOException {
789        statistics.incrementWriteOps(1);
790    
791        final HttpOpParam.Op op = PostOpParam.Op.APPEND;
792        return new Runner(op, f, new BufferSizeParam(bufferSize))
793          .run()
794          .write(bufferSize);
795      }
796    
797      @SuppressWarnings("deprecation")
798      @Override
799      public boolean delete(final Path f) throws IOException {
800        return delete(f, true);
801      }
802    
803      @Override
804      public boolean delete(Path f, boolean recursive) throws IOException {
805        final HttpOpParam.Op op = DeleteOpParam.Op.DELETE;
806        final Map<?, ?> json = run(op, f, new RecursiveParam(recursive));
807        return (Boolean)json.get("boolean");
808      }
809    
810      @Override
811      public FSDataInputStream open(final Path f, final int buffersize
812          ) throws IOException {
813        statistics.incrementReadOps(1);
814        final HttpOpParam.Op op = GetOpParam.Op.OPEN;
815        final URL url = toUrl(op, f, new BufferSizeParam(buffersize));
816        return new FSDataInputStream(new OffsetUrlInputStream(
817            new OffsetUrlOpener(url), new OffsetUrlOpener(null)));
818      }
819    
820      @Override
821      public void close() throws IOException {
822        super.close();
823        if (dtRenewer != null) {
824          dtRenewer.removeRenewAction(this); // blocks
825        }
826      }
827    
828      class OffsetUrlOpener extends ByteRangeInputStream.URLOpener {
829        OffsetUrlOpener(final URL url) {
830          super(url);
831        }
832    
833        /** Setup offset url and connect. */
834        @Override
835        protected HttpURLConnection connect(final long offset,
836            final boolean resolved) throws IOException {
837          final URL offsetUrl = offset == 0L? url
838              : new URL(url + "&" + new OffsetParam(offset));
839          return new Runner(GetOpParam.Op.OPEN, offsetUrl, resolved).run().conn;
840        }  
841      }
842    
843      private static final String OFFSET_PARAM_PREFIX = OffsetParam.NAME + "=";
844    
845      /** Remove offset parameter, if there is any, from the url */
846      static URL removeOffsetParam(final URL url) throws MalformedURLException {
847        String query = url.getQuery();
848        if (query == null) {
849          return url;
850        }
851        final String lower = query.toLowerCase();
852        if (!lower.startsWith(OFFSET_PARAM_PREFIX)
853            && !lower.contains("&" + OFFSET_PARAM_PREFIX)) {
854          return url;
855        }
856    
857        //rebuild query
858        StringBuilder b = null;
859        for(final StringTokenizer st = new StringTokenizer(query, "&");
860            st.hasMoreTokens();) {
861          final String token = st.nextToken();
862          if (!token.toLowerCase().startsWith(OFFSET_PARAM_PREFIX)) {
863            if (b == null) {
864              b = new StringBuilder("?").append(token);
865            } else {
866              b.append('&').append(token);
867            }
868          }
869        }
870        query = b == null? "": b.toString();
871    
872        final String urlStr = url.toString();
873        return new URL(urlStr.substring(0, urlStr.indexOf('?')) + query);
874      }
875    
876      static class OffsetUrlInputStream extends ByteRangeInputStream {
877        OffsetUrlInputStream(OffsetUrlOpener o, OffsetUrlOpener r) {
878          super(o, r);
879        }
880    
881        /** Remove offset parameter before returning the resolved url. */
882        @Override
883        protected URL getResolvedUrl(final HttpURLConnection connection
884            ) throws MalformedURLException {
885          return removeOffsetParam(connection.getURL());
886        }
887      }
888    
889      @Override
890      public FileStatus[] listStatus(final Path f) throws IOException {
891        statistics.incrementReadOps(1);
892    
893        final HttpOpParam.Op op = GetOpParam.Op.LISTSTATUS;
894        final Map<?, ?> json  = run(op, f);
895        final Map<?, ?> rootmap = (Map<?, ?>)json.get(FileStatus.class.getSimpleName() + "es");
896        final Object[] array = (Object[])rootmap.get(FileStatus.class.getSimpleName());
897    
898        //convert FileStatus
899        final FileStatus[] statuses = new FileStatus[array.length];
900        for(int i = 0; i < array.length; i++) {
901          final Map<?, ?> m = (Map<?, ?>)array[i];
902          statuses[i] = makeQualified(JsonUtil.toFileStatus(m, false), f);
903        }
904        return statuses;
905      }
906    
907      @Override
908      public Token<DelegationTokenIdentifier> getDelegationToken(
909          final String renewer) throws IOException {
910        final HttpOpParam.Op op = GetOpParam.Op.GETDELEGATIONTOKEN;
911        final Map<?, ?> m = run(op, null, new RenewerParam(renewer));
912        final Token<DelegationTokenIdentifier> token = JsonUtil.toDelegationToken(m); 
913        SecurityUtil.setTokenService(token, nnAddr);
914        return token;
915      }
916    
917      @Override
918      public Token<?> getRenewToken() {
919        return delegationToken;
920      }
921    
922      @Override
923      public <T extends TokenIdentifier> void setDelegationToken(
924          final Token<T> token) {
925        synchronized(this) {
926          delegationToken = token;
927        }
928      }
929    
930      private synchronized long renewDelegationToken(final Token<?> token
931          ) throws IOException {
932        final HttpOpParam.Op op = PutOpParam.Op.RENEWDELEGATIONTOKEN;
933        TokenArgumentParam dtargParam = new TokenArgumentParam(
934            token.encodeToUrlString());
935        final Map<?, ?> m = run(op, null, dtargParam);
936        return (Long) m.get("long");
937      }
938    
939      private synchronized void cancelDelegationToken(final Token<?> token
940          ) throws IOException {
941        final HttpOpParam.Op op = PutOpParam.Op.CANCELDELEGATIONTOKEN;
942        TokenArgumentParam dtargParam = new TokenArgumentParam(
943            token.encodeToUrlString());
944        run(op, null, dtargParam);
945      }
946      
947      @Override
948      public BlockLocation[] getFileBlockLocations(final FileStatus status,
949          final long offset, final long length) throws IOException {
950        if (status == null) {
951          return null;
952        }
953        return getFileBlockLocations(status.getPath(), offset, length);
954      }
955    
956      @Override
957      public BlockLocation[] getFileBlockLocations(final Path p, 
958          final long offset, final long length) throws IOException {
959        statistics.incrementReadOps(1);
960    
961        final HttpOpParam.Op op = GetOpParam.Op.GET_BLOCK_LOCATIONS;
962        final Map<?, ?> m = run(op, p, new OffsetParam(offset),
963            new LengthParam(length));
964        return DFSUtil.locatedBlocks2Locations(JsonUtil.toLocatedBlocks(m));
965      }
966    
967      @Override
968      public ContentSummary getContentSummary(final Path p) throws IOException {
969        statistics.incrementReadOps(1);
970    
971        final HttpOpParam.Op op = GetOpParam.Op.GETCONTENTSUMMARY;
972        final Map<?, ?> m = run(op, p);
973        return JsonUtil.toContentSummary(m);
974      }
975    
976      @Override
977      public MD5MD5CRC32FileChecksum getFileChecksum(final Path p
978          ) throws IOException {
979        statistics.incrementReadOps(1);
980      
981        final HttpOpParam.Op op = GetOpParam.Op.GETFILECHECKSUM;
982        final Map<?, ?> m = run(op, p);
983        return JsonUtil.toMD5MD5CRC32FileChecksum(m);
984      }
985    
986      /** Delegation token renewer. */
987      public static class DtRenewer extends TokenRenewer {
988        @Override
989        public boolean handleKind(Text kind) {
990          return kind.equals(TOKEN_KIND);
991        }
992      
993        @Override
994        public boolean isManaged(Token<?> token) throws IOException {
995          return true;
996        }
997    
998        private static WebHdfsFileSystem getWebHdfs(
999            final Token<?> token, final Configuration conf) throws IOException {
1000          
1001          final InetSocketAddress nnAddr = SecurityUtil.getTokenServiceAddr(token);
1002          final URI uri = DFSUtil.createUri(WebHdfsFileSystem.SCHEME, nnAddr);
1003          return (WebHdfsFileSystem)FileSystem.get(uri, conf);
1004        }
1005    
1006        @Override
1007        public long renew(final Token<?> token, final Configuration conf
1008            ) throws IOException, InterruptedException {
1009          return getWebHdfs(token, conf).renewDelegationToken(token);
1010        }
1011      
1012        @Override
1013        public void cancel(final Token<?> token, final Configuration conf
1014            ) throws IOException, InterruptedException {
1015          getWebHdfs(token, conf).cancelDelegationToken(token);
1016        }
1017      }
1018      
1019      private static class WebHdfsDelegationTokenSelector
1020      extends AbstractDelegationTokenSelector<DelegationTokenIdentifier> {
1021        private static final DelegationTokenSelector hdfsTokenSelector =
1022            new DelegationTokenSelector();
1023        
1024        public WebHdfsDelegationTokenSelector() {
1025          super(TOKEN_KIND);
1026        }
1027        
1028        Token<DelegationTokenIdentifier> selectToken(URI nnUri,
1029            Collection<Token<?>> tokens, Configuration conf) {
1030          Token<DelegationTokenIdentifier> token =
1031              selectToken(SecurityUtil.buildTokenService(nnUri), tokens);
1032          if (token == null) {
1033            token = hdfsTokenSelector.selectToken(nnUri, tokens, conf); 
1034          }
1035          return token;
1036        }
1037      }
1038    }