001/**
002res * 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
019package org.apache.hadoop.hdfs.web;
020
021import java.io.BufferedInputStream;
022import java.io.BufferedOutputStream;
023import java.io.EOFException;
024import java.io.FileNotFoundException;
025import java.io.IOException;
026import java.io.InputStream;
027import java.lang.reflect.InvocationTargetException;
028import java.net.HttpURLConnection;
029import java.net.InetSocketAddress;
030import java.net.MalformedURLException;
031import java.net.URI;
032import java.net.URL;
033import java.security.PrivilegedExceptionAction;
034import java.util.ArrayList;
035import java.util.EnumSet;
036import java.util.List;
037import java.util.Map;
038import java.util.StringTokenizer;
039
040import javax.ws.rs.core.HttpHeaders;
041import javax.ws.rs.core.MediaType;
042
043import org.apache.commons.io.IOUtils;
044import org.apache.commons.io.input.BoundedInputStream;
045import org.apache.commons.logging.Log;
046import org.apache.commons.logging.LogFactory;
047import org.apache.hadoop.conf.Configuration;
048import org.apache.hadoop.fs.BlockLocation;
049import org.apache.hadoop.fs.CommonConfigurationKeys;
050import org.apache.hadoop.fs.ContentSummary;
051import org.apache.hadoop.fs.DelegationTokenRenewer;
052import org.apache.hadoop.fs.FSDataInputStream;
053import org.apache.hadoop.fs.FSDataOutputStream;
054import org.apache.hadoop.fs.FSInputStream;
055import org.apache.hadoop.fs.FileStatus;
056import org.apache.hadoop.fs.FileSystem;
057import org.apache.hadoop.fs.MD5MD5CRC32FileChecksum;
058import org.apache.hadoop.fs.Options;
059import org.apache.hadoop.fs.Path;
060import org.apache.hadoop.fs.XAttrCodec;
061import org.apache.hadoop.fs.XAttrSetFlag;
062import org.apache.hadoop.fs.permission.AclEntry;
063import org.apache.hadoop.fs.permission.AclStatus;
064import org.apache.hadoop.fs.permission.FsAction;
065import org.apache.hadoop.fs.permission.FsPermission;
066import org.apache.hadoop.hdfs.DFSConfigKeys;
067import org.apache.hadoop.hdfs.DFSUtil;
068import org.apache.hadoop.hdfs.HAUtil;
069import org.apache.hadoop.hdfs.protocol.HdfsFileStatus;
070import org.apache.hadoop.hdfs.security.token.delegation.DelegationTokenIdentifier;
071import org.apache.hadoop.hdfs.server.namenode.SafeModeException;
072import org.apache.hadoop.hdfs.web.resources.*;
073import org.apache.hadoop.hdfs.web.resources.HttpOpParam.Op;
074import org.apache.hadoop.io.Text;
075import org.apache.hadoop.io.retry.RetryPolicies;
076import org.apache.hadoop.io.retry.RetryPolicy;
077import org.apache.hadoop.io.retry.RetryUtils;
078import org.apache.hadoop.ipc.RemoteException;
079import org.apache.hadoop.net.NetUtils;
080import org.apache.hadoop.security.AccessControlException;
081import org.apache.hadoop.security.SecurityUtil;
082import org.apache.hadoop.security.UserGroupInformation;
083import org.apache.hadoop.security.token.SecretManager.InvalidToken;
084import org.apache.hadoop.security.token.Token;
085import org.apache.hadoop.security.token.TokenIdentifier;
086import org.apache.hadoop.security.token.TokenSelector;
087import org.apache.hadoop.security.token.delegation.AbstractDelegationTokenSelector;
088import org.apache.hadoop.util.Progressable;
089import org.apache.hadoop.util.StringUtils;
090import org.codehaus.jackson.map.ObjectMapper;
091import org.codehaus.jackson.map.ObjectReader;
092
093import com.google.common.annotations.VisibleForTesting;
094import com.google.common.base.Preconditions;
095import com.google.common.collect.Lists;
096
097/** A FileSystem for HDFS over the web. */
098public class WebHdfsFileSystem extends FileSystem
099    implements DelegationTokenRenewer.Renewable, TokenAspect.TokenManagementDelegator {
100  public static final Log LOG = LogFactory.getLog(WebHdfsFileSystem.class);
101  /** File System URI: {SCHEME}://namenode:port/path/to/file */
102  public static final String SCHEME = "webhdfs";
103  /** WebHdfs version. */
104  public static final int VERSION = 1;
105  /** Http URI: http://namenode:port/{PATH_PREFIX}/path/to/file */
106  public static final String PATH_PREFIX = "/" + SCHEME + "/v" + VERSION;
107
108  /** Default connection factory may be overridden in tests to use smaller timeout values */
109  protected URLConnectionFactory connectionFactory;
110
111  /** Delegation token kind */
112  public static final Text TOKEN_KIND = new Text("WEBHDFS delegation");
113
114  @VisibleForTesting
115  public static final String CANT_FALLBACK_TO_INSECURE_MSG =
116      "The client is configured to only allow connecting to secure cluster";
117
118  private boolean canRefreshDelegationToken;
119
120  private UserGroupInformation ugi;
121  private URI uri;
122  private Token<?> delegationToken;
123  protected Text tokenServiceName;
124  private RetryPolicy retryPolicy = null;
125  private Path workingDir;
126  private InetSocketAddress nnAddrs[];
127  private int currentNNAddrIndex;
128  private boolean disallowFallbackToInsecureCluster;
129  private static final ObjectReader READER =
130      new ObjectMapper().reader(Map.class);
131
132  /**
133   * Return the protocol scheme for the FileSystem.
134   * <p/>
135   *
136   * @return <code>webhdfs</code>
137   */
138  @Override
139  public String getScheme() {
140    return SCHEME;
141  }
142
143  /**
144   * return the underlying transport protocol (http / https).
145   */
146  protected String getTransportScheme() {
147    return "http";
148  }
149
150  protected Text getTokenKind() {
151    return TOKEN_KIND;
152  }
153
154  @Override
155  public synchronized void initialize(URI uri, Configuration conf
156      ) throws IOException {
157    super.initialize(uri, conf);
158    setConf(conf);
159    /** set user pattern based on configuration file */
160    UserParam.setUserPattern(conf.get(
161        DFSConfigKeys.DFS_WEBHDFS_USER_PATTERN_KEY,
162        DFSConfigKeys.DFS_WEBHDFS_USER_PATTERN_DEFAULT));
163
164    connectionFactory = URLConnectionFactory
165        .newDefaultURLConnectionFactory(conf);
166
167    ugi = UserGroupInformation.getCurrentUser();
168    this.uri = URI.create(uri.getScheme() + "://" + uri.getAuthority());
169    this.nnAddrs = resolveNNAddr();
170
171    boolean isHA = HAUtil.isClientFailoverConfigured(conf, this.uri);
172    boolean isLogicalUri = isHA && HAUtil.isLogicalUri(conf, this.uri);
173    // In non-HA or non-logical URI case, the code needs to call
174    // getCanonicalUri() in order to handle the case where no port is
175    // specified in the URI
176    this.tokenServiceName = isLogicalUri ?
177        HAUtil.buildTokenServiceForLogicalUri(uri, getScheme())
178        : SecurityUtil.buildTokenService(getCanonicalUri());
179
180    if (!isHA) {
181      this.retryPolicy =
182          RetryUtils.getDefaultRetryPolicy(
183              conf,
184              DFSConfigKeys.DFS_HTTP_CLIENT_RETRY_POLICY_ENABLED_KEY,
185              DFSConfigKeys.DFS_HTTP_CLIENT_RETRY_POLICY_ENABLED_DEFAULT,
186              DFSConfigKeys.DFS_HTTP_CLIENT_RETRY_POLICY_SPEC_KEY,
187              DFSConfigKeys.DFS_HTTP_CLIENT_RETRY_POLICY_SPEC_DEFAULT,
188              SafeModeException.class);
189    } else {
190
191      int maxFailoverAttempts = conf.getInt(
192          DFSConfigKeys.DFS_HTTP_CLIENT_FAILOVER_MAX_ATTEMPTS_KEY,
193          DFSConfigKeys.DFS_HTTP_CLIENT_FAILOVER_MAX_ATTEMPTS_DEFAULT);
194      int maxRetryAttempts = conf.getInt(
195          DFSConfigKeys.DFS_HTTP_CLIENT_RETRY_MAX_ATTEMPTS_KEY,
196          DFSConfigKeys.DFS_HTTP_CLIENT_RETRY_MAX_ATTEMPTS_DEFAULT);
197      int failoverSleepBaseMillis = conf.getInt(
198          DFSConfigKeys.DFS_HTTP_CLIENT_FAILOVER_SLEEPTIME_BASE_KEY,
199          DFSConfigKeys.DFS_HTTP_CLIENT_FAILOVER_SLEEPTIME_BASE_DEFAULT);
200      int failoverSleepMaxMillis = conf.getInt(
201          DFSConfigKeys.DFS_HTTP_CLIENT_FAILOVER_SLEEPTIME_MAX_KEY,
202          DFSConfigKeys.DFS_HTTP_CLIENT_FAILOVER_SLEEPTIME_MAX_DEFAULT);
203
204      this.retryPolicy = RetryPolicies
205          .failoverOnNetworkException(RetryPolicies.TRY_ONCE_THEN_FAIL,
206              maxFailoverAttempts, maxRetryAttempts, failoverSleepBaseMillis,
207              failoverSleepMaxMillis);
208    }
209
210    this.workingDir = getHomeDirectory();
211    this.canRefreshDelegationToken = UserGroupInformation.isSecurityEnabled();
212    this.disallowFallbackToInsecureCluster = !conf.getBoolean(
213        CommonConfigurationKeys.IPC_CLIENT_FALLBACK_TO_SIMPLE_AUTH_ALLOWED_KEY,
214        CommonConfigurationKeys.IPC_CLIENT_FALLBACK_TO_SIMPLE_AUTH_ALLOWED_DEFAULT);
215    this.delegationToken = null;
216  }
217
218  @Override
219  public URI getCanonicalUri() {
220    return super.getCanonicalUri();
221  }
222
223  /** Is WebHDFS enabled in conf? */
224  public static boolean isEnabled(final Configuration conf, final Log log) {
225    final boolean b = conf.getBoolean(DFSConfigKeys.DFS_WEBHDFS_ENABLED_KEY,
226        DFSConfigKeys.DFS_WEBHDFS_ENABLED_DEFAULT);
227    return b;
228  }
229
230  TokenSelector<DelegationTokenIdentifier> tokenSelector =
231      new AbstractDelegationTokenSelector<DelegationTokenIdentifier>(getTokenKind()){};
232
233  // the first getAuthParams() for a non-token op will either get the
234  // internal token from the ugi or lazy fetch one
235  protected synchronized Token<?> getDelegationToken() throws IOException {
236    if (canRefreshDelegationToken && delegationToken == null) {
237      Token<?> token = tokenSelector.selectToken(
238          new Text(getCanonicalServiceName()), ugi.getTokens());
239      // ugi tokens are usually indicative of a task which can't
240      // refetch tokens.  even if ugi has credentials, don't attempt
241      // to get another token to match hdfs/rpc behavior
242      if (token != null) {
243        LOG.debug("Using UGI token: " + token);
244        canRefreshDelegationToken = false; 
245      } else {
246        token = getDelegationToken(null);
247        if (token != null) {
248          LOG.debug("Fetched new token: " + token);
249        } else { // security is disabled
250          canRefreshDelegationToken = false;
251        }
252      }
253      setDelegationToken(token);
254    }
255    return delegationToken;
256  }
257
258  @VisibleForTesting
259  synchronized boolean replaceExpiredDelegationToken() throws IOException {
260    boolean replaced = false;
261    if (canRefreshDelegationToken) {
262      Token<?> token = getDelegationToken(null);
263      LOG.debug("Replaced expired token: " + token);
264      setDelegationToken(token);
265      replaced = (token != null);
266    }
267    return replaced;
268  }
269
270  @Override
271  @VisibleForTesting
272  public int getDefaultPort() {
273    return getConf().getInt(DFSConfigKeys.DFS_NAMENODE_HTTP_PORT_KEY,
274        DFSConfigKeys.DFS_NAMENODE_HTTP_PORT_DEFAULT);
275  }
276
277  @Override
278  public URI getUri() {
279    return this.uri;
280  }
281  
282  @Override
283  protected URI canonicalizeUri(URI uri) {
284    return NetUtils.getCanonicalUri(uri, getDefaultPort());
285  }
286
287  /** @return the home directory. */
288  public static String getHomeDirectoryString(final UserGroupInformation ugi) {
289    return "/user/" + ugi.getShortUserName();
290  }
291
292  @Override
293  public Path getHomeDirectory() {
294    return makeQualified(new Path(getHomeDirectoryString(ugi)));
295  }
296
297  @Override
298  public synchronized Path getWorkingDirectory() {
299    return workingDir;
300  }
301
302  @Override
303  public synchronized void setWorkingDirectory(final Path dir) {
304    String result = makeAbsolute(dir).toUri().getPath();
305    if (!DFSUtil.isValidName(result)) {
306      throw new IllegalArgumentException("Invalid DFS directory name " + 
307                                         result);
308    }
309    workingDir = makeAbsolute(dir);
310  }
311
312  private Path makeAbsolute(Path f) {
313    return f.isAbsolute()? f: new Path(workingDir, f);
314  }
315
316  static Map<?, ?> jsonParse(final HttpURLConnection c, final boolean useErrorStream
317      ) throws IOException {
318    if (c.getContentLength() == 0) {
319      return null;
320    }
321    final InputStream in = useErrorStream? c.getErrorStream(): c.getInputStream();
322    if (in == null) {
323      throw new IOException("The " + (useErrorStream? "error": "input") + " stream is null.");
324    }
325    try {
326      final String contentType = c.getContentType();
327      if (contentType != null) {
328        final MediaType parsed = MediaType.valueOf(contentType);
329        if (!MediaType.APPLICATION_JSON_TYPE.isCompatible(parsed)) {
330          throw new IOException("Content-Type \"" + contentType
331              + "\" is incompatible with \"" + MediaType.APPLICATION_JSON
332              + "\" (parsed=\"" + parsed + "\")");
333        }
334      }
335      return READER.readValue(in);
336    } finally {
337      in.close();
338    }
339  }
340
341  private static Map<?, ?> validateResponse(final HttpOpParam.Op op,
342      final HttpURLConnection conn, boolean unwrapException) throws IOException {
343    final int code = conn.getResponseCode();
344    // server is demanding an authentication we don't support
345    if (code == HttpURLConnection.HTTP_UNAUTHORIZED) {
346      // match hdfs/rpc exception
347      throw new AccessControlException(conn.getResponseMessage());
348    }
349    if (code != op.getExpectedHttpResponseCode()) {
350      final Map<?, ?> m;
351      try {
352        m = jsonParse(conn, true);
353      } catch(Exception e) {
354        throw new IOException("Unexpected HTTP response: code=" + code + " != "
355            + op.getExpectedHttpResponseCode() + ", " + op.toQueryString()
356            + ", message=" + conn.getResponseMessage(), e);
357      }
358
359      if (m == null) {
360        throw new IOException("Unexpected HTTP response: code=" + code + " != "
361            + op.getExpectedHttpResponseCode() + ", " + op.toQueryString()
362            + ", message=" + conn.getResponseMessage());
363      } else if (m.get(RemoteException.class.getSimpleName()) == null) {
364        return m;
365      }
366
367      IOException re = JsonUtil.toRemoteException(m);
368      // extract UGI-related exceptions and unwrap InvalidToken
369      // the NN mangles these exceptions but the DN does not and may need
370      // to re-fetch a token if either report the token is expired
371      if (re.getMessage() != null && re.getMessage().startsWith(
372          SecurityUtil.FAILED_TO_GET_UGI_MSG_HEADER)) {
373        String[] parts = re.getMessage().split(":\\s+", 3);
374        re = new RemoteException(parts[1], parts[2]);
375        re = ((RemoteException)re).unwrapRemoteException(InvalidToken.class);
376      }
377      throw unwrapException? toIOException(re): re;
378    }
379    return null;
380  }
381
382  /**
383   * Covert an exception to an IOException.
384   * 
385   * For a non-IOException, wrap it with IOException.
386   * For a RemoteException, unwrap it.
387   * For an IOException which is not a RemoteException, return it. 
388   */
389  private static IOException toIOException(Exception e) {
390    if (!(e instanceof IOException)) {
391      return new IOException(e);
392    }
393
394    final IOException ioe = (IOException)e;
395    if (!(ioe instanceof RemoteException)) {
396      return ioe;
397    }
398
399    return ((RemoteException)ioe).unwrapRemoteException();
400  }
401
402  private synchronized InetSocketAddress getCurrentNNAddr() {
403    return nnAddrs[currentNNAddrIndex];
404  }
405
406  /**
407   * Reset the appropriate state to gracefully fail over to another name node
408   */
409  private synchronized void resetStateToFailOver() {
410    currentNNAddrIndex = (currentNNAddrIndex + 1) % nnAddrs.length;
411  }
412
413  /**
414   * Return a URL pointing to given path on the namenode.
415   *
416   * @param path to obtain the URL for
417   * @param query string to append to the path
418   * @return namenode URL referring to the given path
419   * @throws IOException on error constructing the URL
420   */
421  private URL getNamenodeURL(String path, String query) throws IOException {
422    InetSocketAddress nnAddr = getCurrentNNAddr();
423    final URL url = new URL(getTransportScheme(), nnAddr.getHostName(),
424          nnAddr.getPort(), path + '?' + query);
425    if (LOG.isTraceEnabled()) {
426      LOG.trace("url=" + url);
427    }
428    return url;
429  }
430  
431  Param<?,?>[] getAuthParameters(final HttpOpParam.Op op) throws IOException {
432    List<Param<?,?>> authParams = Lists.newArrayList();    
433    // Skip adding delegation token for token operations because these
434    // operations require authentication.
435    Token<?> token = null;
436    if (!op.getRequireAuth()) {
437      token = getDelegationToken();
438    }
439    if (token != null) {
440      authParams.add(new DelegationParam(token.encodeToUrlString()));
441    } else {
442      UserGroupInformation userUgi = ugi;
443      UserGroupInformation realUgi = userUgi.getRealUser();
444      if (realUgi != null) { // proxy user
445        authParams.add(new DoAsParam(userUgi.getShortUserName()));
446        userUgi = realUgi;
447      }
448      authParams.add(new UserParam(userUgi.getShortUserName()));
449    }
450    return authParams.toArray(new Param<?,?>[0]);
451  }
452
453  URL toUrl(final HttpOpParam.Op op, final Path fspath,
454      final Param<?,?>... parameters) throws IOException {
455    //initialize URI path and query
456    final String path = PATH_PREFIX
457        + (fspath == null? "/": makeQualified(fspath).toUri().getRawPath());
458    final String query = op.toQueryString()
459        + Param.toSortedString("&", getAuthParameters(op))
460        + Param.toSortedString("&", parameters);
461    final URL url = getNamenodeURL(path, query);
462    if (LOG.isTraceEnabled()) {
463      LOG.trace("url=" + url);
464    }
465    return url;
466  }
467
468  /**
469   * This class is for initialing a HTTP connection, connecting to server,
470   * obtaining a response, and also handling retry on failures.
471   */
472  abstract class AbstractRunner<T> {
473    abstract protected URL getUrl() throws IOException;
474
475    protected final HttpOpParam.Op op;
476    private final boolean redirected;
477    protected ExcludeDatanodesParam excludeDatanodes = new ExcludeDatanodesParam("");
478
479    private boolean checkRetry;
480    private String redirectHost;
481
482    protected AbstractRunner(final HttpOpParam.Op op, boolean redirected) {
483      this.op = op;
484      this.redirected = redirected;
485    }
486
487    T run() throws IOException {
488      UserGroupInformation connectUgi = ugi.getRealUser();
489      if (connectUgi == null) {
490        connectUgi = ugi;
491      }
492      if (op.getRequireAuth()) {
493        connectUgi.checkTGTAndReloginFromKeytab();
494      }
495      try {
496        // the entire lifecycle of the connection must be run inside the
497        // doAs to ensure authentication is performed correctly
498        return connectUgi.doAs(
499            new PrivilegedExceptionAction<T>() {
500              @Override
501              public T run() throws IOException {
502                return runWithRetry();
503              }
504            });
505      } catch (InterruptedException e) {
506        throw new IOException(e);
507      }
508    }
509
510    /**
511     * Two-step requests redirected to a DN
512     * 
513     * Create/Append:
514     * Step 1) Submit a Http request with neither auto-redirect nor data. 
515     * Step 2) Submit another Http request with the URL from the Location header with data.
516     * 
517     * The reason of having two-step create/append is for preventing clients to
518     * send out the data before the redirect. This issue is addressed by the
519     * "Expect: 100-continue" header in HTTP/1.1; see RFC 2616, Section 8.2.3.
520     * Unfortunately, there are software library bugs (e.g. Jetty 6 http server
521     * and Java 6 http client), which do not correctly implement "Expect:
522     * 100-continue". The two-step create/append is a temporary workaround for
523     * the software library bugs.
524     * 
525     * Open/Checksum
526     * Also implements two-step connects for other operations redirected to
527     * a DN such as open and checksum
528     */
529    protected HttpURLConnection connect(URL url) throws IOException {
530      //redirect hostname and port
531      redirectHost = null;
532
533      
534      // resolve redirects for a DN operation unless already resolved
535      if (op.getRedirect() && !redirected) {
536        final HttpOpParam.Op redirectOp =
537            HttpOpParam.TemporaryRedirectOp.valueOf(op);
538        final HttpURLConnection conn = connect(redirectOp, url);
539        // application level proxy like httpfs might not issue a redirect
540        if (conn.getResponseCode() == op.getExpectedHttpResponseCode()) {
541          return conn;
542        }
543        try {
544          validateResponse(redirectOp, conn, false);
545          url = new URL(conn.getHeaderField("Location"));
546          redirectHost = url.getHost() + ":" + url.getPort();
547        } finally {
548          // TODO: consider not calling conn.disconnect() to allow connection reuse
549          // See http://tinyurl.com/java7-http-keepalive
550          conn.disconnect();
551        }
552      }
553      try {
554        return connect(op, url);
555      } catch (IOException ioe) {
556        if (redirectHost != null) {
557          if (excludeDatanodes.getValue() != null) {
558            excludeDatanodes = new ExcludeDatanodesParam(redirectHost + ","
559                + excludeDatanodes.getValue());
560          } else {
561            excludeDatanodes = new ExcludeDatanodesParam(redirectHost);
562          }
563        }
564        throw ioe;
565      }      
566    }
567
568    private HttpURLConnection connect(final HttpOpParam.Op op, final URL url)
569        throws IOException {
570      final HttpURLConnection conn =
571          (HttpURLConnection)connectionFactory.openConnection(url);
572      final boolean doOutput = op.getDoOutput();
573      conn.setRequestMethod(op.getType().toString());
574      conn.setInstanceFollowRedirects(false);
575      switch (op.getType()) {
576        // if not sending a message body for a POST or PUT operation, need
577        // to ensure the server/proxy knows this 
578        case POST:
579        case PUT: {
580          conn.setDoOutput(true);
581          if (!doOutput) {
582            // explicitly setting content-length to 0 won't do spnego!!
583            // opening and closing the stream will send "Content-Length: 0"
584            conn.getOutputStream().close();
585          } else {
586            conn.setRequestProperty("Content-Type",
587                MediaType.APPLICATION_OCTET_STREAM);
588            conn.setChunkedStreamingMode(32 << 10); //32kB-chunk
589          }
590          break;
591        }
592        default: {
593          conn.setDoOutput(doOutput);
594          break;
595        }
596      }
597      conn.connect();
598      return conn;
599    }
600
601    private T runWithRetry() throws IOException {
602      /**
603       * Do the real work.
604       *
605       * There are three cases that the code inside the loop can throw an
606       * IOException:
607       *
608       * <ul>
609       * <li>The connection has failed (e.g., ConnectException,
610       * @see FailoverOnNetworkExceptionRetry for more details)</li>
611       * <li>The namenode enters the standby state (i.e., StandbyException).</li>
612       * <li>The server returns errors for the command (i.e., RemoteException)</li>
613       * </ul>
614       *
615       * The call to shouldRetry() will conduct the retry policy. The policy
616       * examines the exception and swallows it if it decides to rerun the work.
617       */
618      for(int retry = 0; ; retry++) {
619        checkRetry = !redirected;
620        final URL url = getUrl();
621        try {
622          final HttpURLConnection conn = connect(url);
623          // output streams will validate on close
624          if (!op.getDoOutput()) {
625            validateResponse(op, conn, false);
626          }
627          return getResponse(conn);
628        } catch (AccessControlException ace) {
629          // no retries for auth failures
630          throw ace;
631        } catch (InvalidToken it) {
632          // try to replace the expired token with a new one.  the attempt
633          // to acquire a new token must be outside this operation's retry
634          // so if it fails after its own retries, this operation fails too.
635          if (op.getRequireAuth() || !replaceExpiredDelegationToken()) {
636            throw it;
637          }
638        } catch (IOException ioe) {
639          // Attempt to include the redirected node in the exception. If the
640          // attempt to recreate the exception fails, just use the original.
641          String node = redirectHost;
642          if (node == null) {
643            node = url.getAuthority();
644          }
645          try {
646            IOException newIoe = ioe.getClass().getConstructor(String.class)
647                .newInstance(node + ": " + ioe.getMessage());
648            newIoe.setStackTrace(ioe.getStackTrace());
649            ioe = newIoe;
650          } catch (NoSuchMethodException | SecurityException 
651                   | InstantiationException | IllegalAccessException
652                   | IllegalArgumentException | InvocationTargetException e) {
653          }
654          shouldRetry(ioe, retry);
655        }
656      }
657    }
658
659    private void shouldRetry(final IOException ioe, final int retry
660        ) throws IOException {
661      InetSocketAddress nnAddr = getCurrentNNAddr();
662      if (checkRetry) {
663        try {
664          final RetryPolicy.RetryAction a = retryPolicy.shouldRetry(
665              ioe, retry, 0, true);
666
667          boolean isRetry = a.action == RetryPolicy.RetryAction.RetryDecision.RETRY;
668          boolean isFailoverAndRetry =
669              a.action == RetryPolicy.RetryAction.RetryDecision.FAILOVER_AND_RETRY;
670
671          if (isRetry || isFailoverAndRetry) {
672            LOG.info("Retrying connect to namenode: " + nnAddr
673                + ". Already tried " + retry + " time(s); retry policy is "
674                + retryPolicy + ", delay " + a.delayMillis + "ms.");
675
676            if (isFailoverAndRetry) {
677              resetStateToFailOver();
678            }
679
680            Thread.sleep(a.delayMillis);
681            return;
682          }
683        } catch(Exception e) {
684          LOG.warn("Original exception is ", ioe);
685          throw toIOException(e);
686        }
687      }
688      throw toIOException(ioe);
689    }
690
691    abstract T getResponse(HttpURLConnection conn) throws IOException;
692  }
693
694  /**
695   * Abstract base class to handle path-based operations with params
696   */
697  abstract class AbstractFsPathRunner<T> extends AbstractRunner<T> {
698    private final Path fspath;
699    private Param<?,?>[] parameters;
700    
701    AbstractFsPathRunner(final HttpOpParam.Op op, final Path fspath,
702        Param<?,?>... parameters) {
703      super(op, false);
704      this.fspath = fspath;
705      this.parameters = parameters;
706    }
707    
708    AbstractFsPathRunner(final HttpOpParam.Op op, Param<?,?>[] parameters,
709        final Path fspath) {
710      super(op, false);
711      this.fspath = fspath;
712      this.parameters = parameters;
713    }
714
715    protected void updateURLParameters(Param<?, ?>... p) {
716      this.parameters = p;
717    }
718    
719    @Override
720    protected URL getUrl() throws IOException {
721      if (excludeDatanodes.getValue() != null) {
722        Param<?, ?>[] tmpParam = new Param<?, ?>[parameters.length + 1];
723        System.arraycopy(parameters, 0, tmpParam, 0, parameters.length);
724        tmpParam[parameters.length] = excludeDatanodes;
725        return toUrl(op, fspath, tmpParam);
726      } else {
727        return toUrl(op, fspath, parameters);
728      }
729    }
730  }
731
732  /**
733   * Default path-based implementation expects no json response
734   */
735  class FsPathRunner extends AbstractFsPathRunner<Void> {
736    FsPathRunner(Op op, Path fspath, Param<?,?>... parameters) {
737      super(op, fspath, parameters);
738    }
739    
740    @Override
741    Void getResponse(HttpURLConnection conn) throws IOException {
742      return null;
743    }
744  }
745
746  /**
747   * Handle path-based operations with a json response
748   */
749  abstract class FsPathResponseRunner<T> extends AbstractFsPathRunner<T> {
750    FsPathResponseRunner(final HttpOpParam.Op op, final Path fspath,
751        Param<?,?>... parameters) {
752      super(op, fspath, parameters);
753    }
754    
755    FsPathResponseRunner(final HttpOpParam.Op op, Param<?,?>[] parameters,
756        final Path fspath) {
757      super(op, parameters, fspath);
758    }
759    
760    @Override
761    final T getResponse(HttpURLConnection conn) throws IOException {
762      try {
763        final Map<?,?> json = jsonParse(conn, false);
764        if (json == null) {
765          // match exception class thrown by parser
766          throw new IllegalStateException("Missing response");
767        }
768        return decodeResponse(json);
769      } catch (IOException ioe) {
770        throw ioe;
771      } catch (Exception e) { // catch json parser errors
772        final IOException ioe =
773            new IOException("Response decoding failure: "+e.toString(), e);
774        if (LOG.isDebugEnabled()) {
775          LOG.debug(ioe);
776        }
777        throw ioe;
778      } finally {
779        // Don't call conn.disconnect() to allow connection reuse
780        // See http://tinyurl.com/java7-http-keepalive
781        conn.getInputStream().close();
782      }
783    }
784    
785    abstract T decodeResponse(Map<?,?> json) throws IOException;
786  }
787
788  /**
789   * Handle path-based operations with json boolean response
790   */
791  class FsPathBooleanRunner extends FsPathResponseRunner<Boolean> {
792    FsPathBooleanRunner(Op op, Path fspath, Param<?,?>... parameters) {
793      super(op, fspath, parameters);
794    }
795    
796    @Override
797    Boolean decodeResponse(Map<?,?> json) throws IOException {
798      return (Boolean)json.get("boolean");
799    }
800  }
801
802  /**
803   * Handle create/append output streams
804   */
805  class FsPathOutputStreamRunner extends AbstractFsPathRunner<FSDataOutputStream> {
806    private final int bufferSize;
807    
808    FsPathOutputStreamRunner(Op op, Path fspath, int bufferSize,
809        Param<?,?>... parameters) {
810      super(op, fspath, parameters);
811      this.bufferSize = bufferSize;
812    }
813    
814    @Override
815    FSDataOutputStream getResponse(final HttpURLConnection conn)
816        throws IOException {
817      return new FSDataOutputStream(new BufferedOutputStream(
818          conn.getOutputStream(), bufferSize), statistics) {
819        @Override
820        public void close() throws IOException {
821          try {
822            super.close();
823          } finally {
824            try {
825              validateResponse(op, conn, true);
826            } finally {
827              // This is a connection to DataNode.  Let's disconnect since
828              // there is little chance that the connection will be reused
829              // any time soonl
830              conn.disconnect();
831            }
832          }
833        }
834      };
835    }
836  }
837
838  class FsPathConnectionRunner extends AbstractFsPathRunner<HttpURLConnection> {
839    FsPathConnectionRunner(Op op, Path fspath, Param<?,?>... parameters) {
840      super(op, fspath, parameters);
841    }
842    @Override
843    HttpURLConnection getResponse(final HttpURLConnection conn)
844        throws IOException {
845      return conn;
846    }
847  }
848  
849  /**
850   * Used by open() which tracks the resolved url itself
851   */
852  final class URLRunner extends AbstractRunner<HttpURLConnection> {
853    private final URL url;
854    @Override
855    protected URL getUrl() {
856      return url;
857    }
858
859    protected URLRunner(final HttpOpParam.Op op, final URL url, boolean redirected) {
860      super(op, redirected);
861      this.url = url;
862    }
863
864    @Override
865    HttpURLConnection getResponse(HttpURLConnection conn) throws IOException {
866      return conn;
867    }
868  }
869
870  private FsPermission applyUMask(FsPermission permission) {
871    if (permission == null) {
872      permission = FsPermission.getDefault();
873    }
874    return permission.applyUMask(FsPermission.getUMask(getConf()));
875  }
876
877  private HdfsFileStatus getHdfsFileStatus(Path f) throws IOException {
878    final HttpOpParam.Op op = GetOpParam.Op.GETFILESTATUS;
879    HdfsFileStatus status = new FsPathResponseRunner<HdfsFileStatus>(op, f) {
880      @Override
881      HdfsFileStatus decodeResponse(Map<?,?> json) {
882        return JsonUtil.toFileStatus(json, true);
883      }
884    }.run();
885    if (status == null) {
886      throw new FileNotFoundException("File does not exist: " + f);
887    }
888    return status;
889  }
890
891  @Override
892  public FileStatus getFileStatus(Path f) throws IOException {
893    statistics.incrementReadOps(1);
894    return makeQualified(getHdfsFileStatus(f), f);
895  }
896
897  private FileStatus makeQualified(HdfsFileStatus f, Path parent) {
898    return new FileStatus(f.getLen(), f.isDir(), f.getReplication(),
899        f.getBlockSize(), f.getModificationTime(), f.getAccessTime(),
900        f.getPermission(), f.getOwner(), f.getGroup(),
901        f.isSymlink() ? new Path(f.getSymlink()) : null,
902        f.getFullPath(parent).makeQualified(getUri(), getWorkingDirectory()));
903  }
904
905  @Override
906  public AclStatus getAclStatus(Path f) throws IOException {
907    final HttpOpParam.Op op = GetOpParam.Op.GETACLSTATUS;
908    AclStatus status = new FsPathResponseRunner<AclStatus>(op, f) {
909      @Override
910      AclStatus decodeResponse(Map<?,?> json) {
911        return JsonUtil.toAclStatus(json);
912      }
913    }.run();
914    if (status == null) {
915      throw new FileNotFoundException("File does not exist: " + f);
916    }
917    return status;
918  }
919
920  @Override
921  public boolean mkdirs(Path f, FsPermission permission) throws IOException {
922    statistics.incrementWriteOps(1);
923    final HttpOpParam.Op op = PutOpParam.Op.MKDIRS;
924    return new FsPathBooleanRunner(op, f,
925        new PermissionParam(applyUMask(permission))
926    ).run();
927  }
928
929  /**
930   * Create a symlink pointing to the destination path.
931   * @see org.apache.hadoop.fs.Hdfs#createSymlink(Path, Path, boolean) 
932   */
933  public void createSymlink(Path destination, Path f, boolean createParent
934      ) throws IOException {
935    statistics.incrementWriteOps(1);
936    final HttpOpParam.Op op = PutOpParam.Op.CREATESYMLINK;
937    new FsPathRunner(op, f,
938        new DestinationParam(makeQualified(destination).toUri().getPath()),
939        new CreateParentParam(createParent)
940    ).run();
941  }
942
943  @Override
944  public boolean rename(final Path src, final Path dst) throws IOException {
945    statistics.incrementWriteOps(1);
946    final HttpOpParam.Op op = PutOpParam.Op.RENAME;
947    return new FsPathBooleanRunner(op, src,
948        new DestinationParam(makeQualified(dst).toUri().getPath())
949    ).run();
950  }
951
952  @SuppressWarnings("deprecation")
953  @Override
954  public void rename(final Path src, final Path dst,
955      final Options.Rename... options) throws IOException {
956    statistics.incrementWriteOps(1);
957    final HttpOpParam.Op op = PutOpParam.Op.RENAME;
958    new FsPathRunner(op, src,
959        new DestinationParam(makeQualified(dst).toUri().getPath()),
960        new RenameOptionSetParam(options)
961    ).run();
962  }
963  
964  @Override
965  public void setXAttr(Path p, String name, byte[] value, 
966      EnumSet<XAttrSetFlag> flag) throws IOException {
967    statistics.incrementWriteOps(1);
968    final HttpOpParam.Op op = PutOpParam.Op.SETXATTR;
969    if (value != null) {
970      new FsPathRunner(op, p, new XAttrNameParam(name), new XAttrValueParam(
971          XAttrCodec.encodeValue(value, XAttrCodec.HEX)), 
972          new XAttrSetFlagParam(flag)).run();
973    } else {
974      new FsPathRunner(op, p, new XAttrNameParam(name), 
975          new XAttrSetFlagParam(flag)).run();
976    }
977  }
978  
979  @Override
980  public byte[] getXAttr(Path p, final String name) throws IOException {
981    final HttpOpParam.Op op = GetOpParam.Op.GETXATTRS;
982    return new FsPathResponseRunner<byte[]>(op, p, new XAttrNameParam(name), 
983        new XAttrEncodingParam(XAttrCodec.HEX)) {
984      @Override
985      byte[] decodeResponse(Map<?, ?> json) throws IOException {
986        return JsonUtil.getXAttr(json, name);
987      }
988    }.run();
989  }
990  
991  @Override
992  public Map<String, byte[]> getXAttrs(Path p) throws IOException {
993    final HttpOpParam.Op op = GetOpParam.Op.GETXATTRS;
994    return new FsPathResponseRunner<Map<String, byte[]>>(op, p, 
995        new XAttrEncodingParam(XAttrCodec.HEX)) {
996      @Override
997      Map<String, byte[]> decodeResponse(Map<?, ?> json) throws IOException {
998        return JsonUtil.toXAttrs(json);
999      }
1000    }.run();
1001  }
1002  
1003  @Override
1004  public Map<String, byte[]> getXAttrs(Path p, final List<String> names) 
1005      throws IOException {
1006    Preconditions.checkArgument(names != null && !names.isEmpty(), 
1007        "XAttr names cannot be null or empty.");
1008    Param<?,?>[] parameters = new Param<?,?>[names.size() + 1];
1009    for (int i = 0; i < parameters.length - 1; i++) {
1010      parameters[i] = new XAttrNameParam(names.get(i));
1011    }
1012    parameters[parameters.length - 1] = new XAttrEncodingParam(XAttrCodec.HEX);
1013    
1014    final HttpOpParam.Op op = GetOpParam.Op.GETXATTRS;
1015    return new FsPathResponseRunner<Map<String, byte[]>>(op, parameters, p) {
1016      @Override
1017      Map<String, byte[]> decodeResponse(Map<?, ?> json) throws IOException {
1018        return JsonUtil.toXAttrs(json);
1019      }
1020    }.run();
1021  }
1022  
1023  @Override
1024  public List<String> listXAttrs(Path p) throws IOException {
1025    final HttpOpParam.Op op = GetOpParam.Op.LISTXATTRS;
1026    return new FsPathResponseRunner<List<String>>(op, p) {
1027      @Override
1028      List<String> decodeResponse(Map<?, ?> json) throws IOException {
1029        return JsonUtil.toXAttrNames(json);
1030      }
1031    }.run();
1032  }
1033
1034  @Override
1035  public void removeXAttr(Path p, String name) throws IOException {
1036    statistics.incrementWriteOps(1);
1037    final HttpOpParam.Op op = PutOpParam.Op.REMOVEXATTR;
1038    new FsPathRunner(op, p, new XAttrNameParam(name)).run();
1039  }
1040
1041  @Override
1042  public void setOwner(final Path p, final String owner, final String group
1043      ) throws IOException {
1044    if (owner == null && group == null) {
1045      throw new IOException("owner == null && group == null");
1046    }
1047
1048    statistics.incrementWriteOps(1);
1049    final HttpOpParam.Op op = PutOpParam.Op.SETOWNER;
1050    new FsPathRunner(op, p,
1051        new OwnerParam(owner), new GroupParam(group)
1052    ).run();
1053  }
1054
1055  @Override
1056  public void setPermission(final Path p, final FsPermission permission
1057      ) throws IOException {
1058    statistics.incrementWriteOps(1);
1059    final HttpOpParam.Op op = PutOpParam.Op.SETPERMISSION;
1060    new FsPathRunner(op, p,new PermissionParam(permission)).run();
1061  }
1062
1063  @Override
1064  public void modifyAclEntries(Path path, List<AclEntry> aclSpec)
1065      throws IOException {
1066    statistics.incrementWriteOps(1);
1067    final HttpOpParam.Op op = PutOpParam.Op.MODIFYACLENTRIES;
1068    new FsPathRunner(op, path, new AclPermissionParam(aclSpec)).run();
1069  }
1070
1071  @Override
1072  public void removeAclEntries(Path path, List<AclEntry> aclSpec)
1073      throws IOException {
1074    statistics.incrementWriteOps(1);
1075    final HttpOpParam.Op op = PutOpParam.Op.REMOVEACLENTRIES;
1076    new FsPathRunner(op, path, new AclPermissionParam(aclSpec)).run();
1077  }
1078
1079  @Override
1080  public void removeDefaultAcl(Path path) throws IOException {
1081    statistics.incrementWriteOps(1);
1082    final HttpOpParam.Op op = PutOpParam.Op.REMOVEDEFAULTACL;
1083    new FsPathRunner(op, path).run();
1084  }
1085
1086  @Override
1087  public void removeAcl(Path path) throws IOException {
1088    statistics.incrementWriteOps(1);
1089    final HttpOpParam.Op op = PutOpParam.Op.REMOVEACL;
1090    new FsPathRunner(op, path).run();
1091  }
1092
1093  @Override
1094  public void setAcl(final Path p, final List<AclEntry> aclSpec)
1095      throws IOException {
1096    statistics.incrementWriteOps(1);
1097    final HttpOpParam.Op op = PutOpParam.Op.SETACL;
1098    new FsPathRunner(op, p, new AclPermissionParam(aclSpec)).run();
1099  }
1100
1101  @Override
1102  public Path createSnapshot(final Path path, final String snapshotName) 
1103      throws IOException {
1104    statistics.incrementWriteOps(1);
1105    final HttpOpParam.Op op = PutOpParam.Op.CREATESNAPSHOT;
1106    Path spath = new FsPathResponseRunner<Path>(op, path,
1107        new SnapshotNameParam(snapshotName)) {
1108      @Override
1109      Path decodeResponse(Map<?,?> json) {
1110        return new Path((String) json.get(Path.class.getSimpleName()));
1111      }
1112    }.run();
1113    return spath;
1114  }
1115
1116  @Override
1117  public void deleteSnapshot(final Path path, final String snapshotName)
1118      throws IOException {
1119    statistics.incrementWriteOps(1);
1120    final HttpOpParam.Op op = DeleteOpParam.Op.DELETESNAPSHOT;
1121    new FsPathRunner(op, path, new SnapshotNameParam(snapshotName)).run();
1122  }
1123
1124  @Override
1125  public void renameSnapshot(final Path path, final String snapshotOldName,
1126      final String snapshotNewName) throws IOException {
1127    statistics.incrementWriteOps(1);
1128    final HttpOpParam.Op op = PutOpParam.Op.RENAMESNAPSHOT;
1129    new FsPathRunner(op, path, new OldSnapshotNameParam(snapshotOldName),
1130        new SnapshotNameParam(snapshotNewName)).run();
1131  }
1132
1133  @Override
1134  public boolean setReplication(final Path p, final short replication
1135     ) throws IOException {
1136    statistics.incrementWriteOps(1);
1137    final HttpOpParam.Op op = PutOpParam.Op.SETREPLICATION;
1138    return new FsPathBooleanRunner(op, p,
1139        new ReplicationParam(replication)
1140    ).run();
1141  }
1142
1143  @Override
1144  public void setTimes(final Path p, final long mtime, final long atime
1145      ) throws IOException {
1146    statistics.incrementWriteOps(1);
1147    final HttpOpParam.Op op = PutOpParam.Op.SETTIMES;
1148    new FsPathRunner(op, p,
1149        new ModificationTimeParam(mtime),
1150        new AccessTimeParam(atime)
1151    ).run();
1152  }
1153
1154  @Override
1155  public long getDefaultBlockSize() {
1156    return getConf().getLongBytes(DFSConfigKeys.DFS_BLOCK_SIZE_KEY,
1157        DFSConfigKeys.DFS_BLOCK_SIZE_DEFAULT);
1158  }
1159
1160  @Override
1161  public short getDefaultReplication() {
1162    return (short)getConf().getInt(DFSConfigKeys.DFS_REPLICATION_KEY,
1163        DFSConfigKeys.DFS_REPLICATION_DEFAULT);
1164  }
1165
1166  @Override
1167  public void concat(final Path trg, final Path [] srcs) throws IOException {
1168    statistics.incrementWriteOps(1);
1169    final HttpOpParam.Op op = PostOpParam.Op.CONCAT;
1170    new FsPathRunner(op, trg, new ConcatSourcesParam(srcs)).run();
1171  }
1172
1173  @Override
1174  public FSDataOutputStream create(final Path f, final FsPermission permission,
1175      final boolean overwrite, final int bufferSize, final short replication,
1176      final long blockSize, final Progressable progress) throws IOException {
1177    statistics.incrementWriteOps(1);
1178
1179    final HttpOpParam.Op op = PutOpParam.Op.CREATE;
1180    return new FsPathOutputStreamRunner(op, f, bufferSize,
1181        new PermissionParam(applyUMask(permission)),
1182        new OverwriteParam(overwrite),
1183        new BufferSizeParam(bufferSize),
1184        new ReplicationParam(replication),
1185        new BlockSizeParam(blockSize)
1186    ).run();
1187  }
1188
1189  @Override
1190  public FSDataOutputStream append(final Path f, final int bufferSize,
1191      final Progressable progress) throws IOException {
1192    statistics.incrementWriteOps(1);
1193
1194    final HttpOpParam.Op op = PostOpParam.Op.APPEND;
1195    return new FsPathOutputStreamRunner(op, f, bufferSize,
1196        new BufferSizeParam(bufferSize)
1197    ).run();
1198  }
1199
1200  @Override
1201  public boolean truncate(Path f, long newLength) throws IOException {
1202    statistics.incrementWriteOps(1);
1203
1204    final HttpOpParam.Op op = PostOpParam.Op.TRUNCATE;
1205    return new FsPathBooleanRunner(op, f, new NewLengthParam(newLength)).run();
1206  }
1207
1208  @Override
1209  public boolean delete(Path f, boolean recursive) throws IOException {
1210    final HttpOpParam.Op op = DeleteOpParam.Op.DELETE;
1211    return new FsPathBooleanRunner(op, f,
1212        new RecursiveParam(recursive)
1213    ).run();
1214  }
1215
1216  @Override
1217  public FSDataInputStream open(final Path f, final int bufferSize
1218      ) throws IOException {
1219    statistics.incrementReadOps(1);
1220    return new FSDataInputStream(new WebHdfsInputStream(f, bufferSize));
1221  }
1222
1223  @Override
1224  public synchronized void close() throws IOException {
1225    try {
1226      if (canRefreshDelegationToken && delegationToken != null) {
1227        cancelDelegationToken(delegationToken);
1228      }
1229    } catch (IOException ioe) {
1230      LOG.debug("Token cancel failed: "+ioe);
1231    } finally {
1232      super.close();
1233    }
1234  }
1235
1236  // use FsPathConnectionRunner to ensure retries for InvalidTokens
1237  class UnresolvedUrlOpener extends ByteRangeInputStream.URLOpener {
1238    private final FsPathConnectionRunner runner;
1239    UnresolvedUrlOpener(FsPathConnectionRunner runner) {
1240      super(null);
1241      this.runner = runner;
1242    }
1243
1244    @Override
1245    protected HttpURLConnection connect(long offset, boolean resolved)
1246        throws IOException {
1247      assert offset == 0;
1248      HttpURLConnection conn = runner.run();
1249      setURL(conn.getURL());
1250      return conn;
1251    }
1252  }
1253
1254  class OffsetUrlOpener extends ByteRangeInputStream.URLOpener {
1255    OffsetUrlOpener(final URL url) {
1256      super(url);
1257    }
1258
1259    /** Setup offset url and connect. */
1260    @Override
1261    protected HttpURLConnection connect(final long offset,
1262        final boolean resolved) throws IOException {
1263      final URL offsetUrl = offset == 0L? url
1264          : new URL(url + "&" + new OffsetParam(offset));
1265      return new URLRunner(GetOpParam.Op.OPEN, offsetUrl, resolved).run();
1266    }  
1267  }
1268
1269  private static final String OFFSET_PARAM_PREFIX = OffsetParam.NAME + "=";
1270
1271  /** Remove offset parameter, if there is any, from the url */
1272  static URL removeOffsetParam(final URL url) throws MalformedURLException {
1273    String query = url.getQuery();
1274    if (query == null) {
1275      return url;
1276    }
1277    final String lower = StringUtils.toLowerCase(query);
1278    if (!lower.startsWith(OFFSET_PARAM_PREFIX)
1279        && !lower.contains("&" + OFFSET_PARAM_PREFIX)) {
1280      return url;
1281    }
1282
1283    //rebuild query
1284    StringBuilder b = null;
1285    for(final StringTokenizer st = new StringTokenizer(query, "&");
1286        st.hasMoreTokens();) {
1287      final String token = st.nextToken();
1288      if (!StringUtils.toLowerCase(token).startsWith(OFFSET_PARAM_PREFIX)) {
1289        if (b == null) {
1290          b = new StringBuilder("?").append(token);
1291        } else {
1292          b.append('&').append(token);
1293        }
1294      }
1295    }
1296    query = b == null? "": b.toString();
1297
1298    final String urlStr = url.toString();
1299    return new URL(urlStr.substring(0, urlStr.indexOf('?')) + query);
1300  }
1301
1302  static class OffsetUrlInputStream extends ByteRangeInputStream {
1303    OffsetUrlInputStream(UnresolvedUrlOpener o, OffsetUrlOpener r)
1304        throws IOException {
1305      super(o, r);
1306    }
1307
1308    /** Remove offset parameter before returning the resolved url. */
1309    @Override
1310    protected URL getResolvedUrl(final HttpURLConnection connection
1311        ) throws MalformedURLException {
1312      return removeOffsetParam(connection.getURL());
1313    }
1314  }
1315
1316  @Override
1317  public FileStatus[] listStatus(final Path f) throws IOException {
1318    statistics.incrementReadOps(1);
1319
1320    final HttpOpParam.Op op = GetOpParam.Op.LISTSTATUS;
1321    return new FsPathResponseRunner<FileStatus[]>(op, f) {
1322      @Override
1323      FileStatus[] decodeResponse(Map<?,?> json) {
1324        final Map<?, ?> rootmap = (Map<?, ?>)json.get(FileStatus.class.getSimpleName() + "es");
1325        final List<?> array = JsonUtil.getList(
1326            rootmap, FileStatus.class.getSimpleName());
1327
1328        //convert FileStatus
1329        final FileStatus[] statuses = new FileStatus[array.size()];
1330        int i = 0;
1331        for (Object object : array) {
1332          final Map<?, ?> m = (Map<?, ?>) object;
1333          statuses[i++] = makeQualified(JsonUtil.toFileStatus(m, false), f);
1334        }
1335        return statuses;
1336      }
1337    }.run();
1338  }
1339
1340  @Override
1341  public Token<DelegationTokenIdentifier> getDelegationToken(
1342      final String renewer) throws IOException {
1343    final HttpOpParam.Op op = GetOpParam.Op.GETDELEGATIONTOKEN;
1344    Token<DelegationTokenIdentifier> token =
1345        new FsPathResponseRunner<Token<DelegationTokenIdentifier>>(
1346            op, null, new RenewerParam(renewer)) {
1347      @Override
1348      Token<DelegationTokenIdentifier> decodeResponse(Map<?,?> json)
1349          throws IOException {
1350        return JsonUtil.toDelegationToken(json);
1351      }
1352    }.run();
1353    if (token != null) {
1354      token.setService(tokenServiceName);
1355    } else {
1356      if (disallowFallbackToInsecureCluster) {
1357        throw new AccessControlException(CANT_FALLBACK_TO_INSECURE_MSG);
1358      }
1359    }
1360    return token;
1361  }
1362
1363  @Override
1364  public synchronized Token<?> getRenewToken() {
1365    return delegationToken;
1366  }
1367
1368  @Override
1369  public <T extends TokenIdentifier> void setDelegationToken(
1370      final Token<T> token) {
1371    synchronized (this) {
1372      delegationToken = token;
1373    }
1374  }
1375
1376  @Override
1377  public synchronized long renewDelegationToken(final Token<?> token
1378      ) throws IOException {
1379    final HttpOpParam.Op op = PutOpParam.Op.RENEWDELEGATIONTOKEN;
1380    return new FsPathResponseRunner<Long>(op, null,
1381        new TokenArgumentParam(token.encodeToUrlString())) {
1382      @Override
1383      Long decodeResponse(Map<?,?> json) throws IOException {
1384        return ((Number) json.get("long")).longValue();
1385      }
1386    }.run();
1387  }
1388
1389  @Override
1390  public synchronized void cancelDelegationToken(final Token<?> token
1391      ) throws IOException {
1392    final HttpOpParam.Op op = PutOpParam.Op.CANCELDELEGATIONTOKEN;
1393    new FsPathRunner(op, null,
1394        new TokenArgumentParam(token.encodeToUrlString())
1395    ).run();
1396  }
1397  
1398  @Override
1399  public BlockLocation[] getFileBlockLocations(final FileStatus status,
1400      final long offset, final long length) throws IOException {
1401    if (status == null) {
1402      return null;
1403    }
1404    return getFileBlockLocations(status.getPath(), offset, length);
1405  }
1406
1407  @Override
1408  public BlockLocation[] getFileBlockLocations(final Path p, 
1409      final long offset, final long length) throws IOException {
1410    statistics.incrementReadOps(1);
1411
1412    final HttpOpParam.Op op = GetOpParam.Op.GET_BLOCK_LOCATIONS;
1413    return new FsPathResponseRunner<BlockLocation[]>(op, p,
1414        new OffsetParam(offset), new LengthParam(length)) {
1415      @Override
1416      BlockLocation[] decodeResponse(Map<?,?> json) throws IOException {
1417        return DFSUtil.locatedBlocks2Locations(
1418            JsonUtil.toLocatedBlocks(json));
1419      }
1420    }.run();
1421  }
1422
1423  @Override
1424  public void access(final Path path, final FsAction mode) throws IOException {
1425    final HttpOpParam.Op op = GetOpParam.Op.CHECKACCESS;
1426    new FsPathRunner(op, path, new FsActionParam(mode)).run();
1427  }
1428
1429  @Override
1430  public ContentSummary getContentSummary(final Path p) throws IOException {
1431    statistics.incrementReadOps(1);
1432
1433    final HttpOpParam.Op op = GetOpParam.Op.GETCONTENTSUMMARY;
1434    return new FsPathResponseRunner<ContentSummary>(op, p) {
1435      @Override
1436      ContentSummary decodeResponse(Map<?,?> json) {
1437        return JsonUtil.toContentSummary(json);        
1438      }
1439    }.run();
1440  }
1441
1442  @Override
1443  public MD5MD5CRC32FileChecksum getFileChecksum(final Path p
1444      ) throws IOException {
1445    statistics.incrementReadOps(1);
1446  
1447    final HttpOpParam.Op op = GetOpParam.Op.GETFILECHECKSUM;
1448    return new FsPathResponseRunner<MD5MD5CRC32FileChecksum>(op, p) {
1449      @Override
1450      MD5MD5CRC32FileChecksum decodeResponse(Map<?,?> json) throws IOException {
1451        return JsonUtil.toMD5MD5CRC32FileChecksum(json);
1452      }
1453    }.run();
1454  }
1455
1456  /**
1457   * Resolve an HDFS URL into real INetSocketAddress. It works like a DNS
1458   * resolver when the URL points to an non-HA cluster. When the URL points to
1459   * an HA cluster with its logical name, the resolver further resolves the
1460   * logical name(i.e., the authority in the URL) into real namenode addresses.
1461   */
1462  private InetSocketAddress[] resolveNNAddr() throws IOException {
1463    Configuration conf = getConf();
1464    final String scheme = uri.getScheme();
1465
1466    ArrayList<InetSocketAddress> ret = new ArrayList<InetSocketAddress>();
1467
1468    if (!HAUtil.isLogicalUri(conf, uri)) {
1469      InetSocketAddress addr = NetUtils.createSocketAddr(uri.getAuthority(),
1470          getDefaultPort());
1471      ret.add(addr);
1472
1473    } else {
1474      Map<String, Map<String, InetSocketAddress>> addresses = DFSUtil
1475          .getHaNnWebHdfsAddresses(conf, scheme);
1476
1477      // Extract the entry corresponding to the logical name.
1478      Map<String, InetSocketAddress> addrs = addresses.get(uri.getHost());
1479      for (InetSocketAddress addr : addrs.values()) {
1480        ret.add(addr);
1481      }
1482    }
1483
1484    InetSocketAddress[] r = new InetSocketAddress[ret.size()];
1485    return ret.toArray(r);
1486  }
1487
1488  @Override
1489  public String getCanonicalServiceName() {
1490    return tokenServiceName == null ? super.getCanonicalServiceName()
1491        : tokenServiceName.toString();
1492  }
1493
1494  @VisibleForTesting
1495  InetSocketAddress[] getResolvedNNAddr() {
1496    return nnAddrs;
1497  }
1498
1499  @VisibleForTesting
1500  public void setRetryPolicy(RetryPolicy rp) {
1501    this.retryPolicy = rp;
1502  }
1503
1504  /**
1505   * This class is used for opening, reading, and seeking files while using the
1506   * WebHdfsFileSystem. This class will invoke the retry policy when performing
1507   * any of these actions.
1508   */
1509  @VisibleForTesting
1510  public class WebHdfsInputStream extends FSInputStream {
1511    private ReadRunner readRunner = null;
1512
1513    WebHdfsInputStream(Path path, int buffersize) throws IOException {
1514      // Only create the ReadRunner once. Each read's byte array and position
1515      // will be updated within the ReadRunner object before every read.
1516      readRunner = new ReadRunner(path, buffersize);
1517    }
1518
1519    @Override
1520    public int read() throws IOException {
1521      final byte[] b = new byte[1];
1522      return (read(b, 0, 1) == -1) ? -1 : (b[0] & 0xff);
1523    }
1524
1525    @Override
1526    public int read(byte b[], int off, int len) throws IOException {
1527      return readRunner.read(b, off, len);
1528    }
1529
1530    @Override
1531    public void seek(long newPos) throws IOException {
1532      readRunner.seek(newPos);
1533    }
1534
1535    @Override
1536    public long getPos() throws IOException {
1537      return readRunner.getPos();
1538    }
1539
1540    protected int getBufferSize() throws IOException {
1541      return readRunner.getBufferSize();
1542    }
1543
1544    protected Path getPath() throws IOException {
1545      return readRunner.getPath();
1546    }
1547
1548    @Override
1549    public boolean seekToNewSource(long targetPos) throws IOException {
1550      return false;
1551    }
1552
1553    @Override
1554    public void close() throws IOException {
1555      readRunner.close();
1556    }
1557
1558    public void setFileLength(long len) {
1559      readRunner.setFileLength(len);
1560    }
1561
1562    public long getFileLength() {
1563      return readRunner.getFileLength();
1564    }
1565
1566    @VisibleForTesting
1567    ReadRunner getReadRunner() {
1568      return readRunner;
1569    }
1570
1571    @VisibleForTesting
1572    void setReadRunner(ReadRunner rr) {
1573      this.readRunner = rr;
1574    }
1575  }
1576
1577  enum RunnerState {
1578    DISCONNECTED, // Connection is closed programmatically by ReadRunner
1579    OPEN,         // Connection has been established by ReadRunner
1580    SEEK,         // Calling code has explicitly called seek()
1581    CLOSED        // Calling code has explicitly called close()
1582    }
1583
1584  /**
1585   * This class will allow retries to occur for both open and read operations.
1586   * The first WebHdfsFileSystem#open creates a new WebHdfsInputStream object,
1587   * which creates a new ReadRunner object that will be used to open a
1588   * connection and read or seek into the input stream.
1589   *
1590   * ReadRunner is a subclass of the AbstractRunner class, which will run the
1591   * ReadRunner#getUrl(), ReadRunner#connect(URL), and ReadRunner#getResponse
1592   * methods within a retry loop, based on the configured retry policy.
1593   * ReadRunner#connect will create a connection if one has not already been
1594   * created. Otherwise, it will return the previously created connection
1595   * object. This is necessary because a new connection should not be created
1596   * for every read.
1597   * Likewise, ReadRunner#getUrl will construct a new URL object only if the
1598   * connection has not previously been established. Otherwise, it will return
1599   * the previously created URL object.
1600   * ReadRunner#getResponse will initialize the input stream if it has not
1601   * already been initialized and read the requested data from the specified
1602   * input stream.
1603   */
1604  @VisibleForTesting
1605  protected class ReadRunner extends AbstractFsPathRunner<Integer> {
1606    private InputStream in = null;
1607    private HttpURLConnection cachedConnection = null;
1608    private byte[] readBuffer;
1609    private int readOffset;
1610    private int readLength;
1611    private RunnerState runnerState = RunnerState.DISCONNECTED;
1612    private URL originalUrl = null;
1613    private URL resolvedUrl = null;
1614
1615    private final Path path;
1616    private final int bufferSize;
1617    private long pos = 0;
1618    private long fileLength = 0;
1619
1620    /* The following methods are WebHdfsInputStream helpers. */
1621
1622    ReadRunner(Path p, int bs) throws IOException {
1623      super(GetOpParam.Op.OPEN, p, new BufferSizeParam(bs));
1624      this.path = p;
1625      this.bufferSize = bs;
1626    }
1627
1628    int read(byte[] b, int off, int len) throws IOException {
1629      if (runnerState == RunnerState.CLOSED) {
1630        throw new IOException("Stream closed");
1631      }
1632
1633      // Before the first read, pos and fileLength will be 0 and readBuffer
1634      // will all be null. They will be initialized once the first connection
1635      // is made. Only after that it makes sense to compare pos and fileLength.
1636      if (pos >= fileLength && readBuffer != null) {
1637        return -1;
1638      }
1639
1640      // If a seek is occurring, the input stream will have been closed, so it
1641      // needs to be reopened. Use the URLRunner to call AbstractRunner#connect
1642      // with the previously-cached resolved URL and with the 'redirected' flag
1643      // set to 'true'. The resolved URL contains the URL of the previously
1644      // opened DN as opposed to the NN. It is preferable to use the resolved
1645      // URL when creating a connection because it does not hit the NN or every
1646      // seek, nor does it open a connection to a new DN after every seek.
1647      // The redirect flag is needed so that AbstractRunner#connect knows the
1648      // URL is already resolved.
1649      // Note that when the redirected flag is set, retries are not attempted.
1650      // So, if the connection fails using URLRunner, clear out the connection
1651      // and fall through to establish the connection using ReadRunner.
1652      if (runnerState == RunnerState.SEEK) {
1653        try {
1654          final URL rurl = new URL(resolvedUrl + "&" + new OffsetParam(pos));
1655          cachedConnection = new URLRunner(GetOpParam.Op.OPEN, rurl, true).run();
1656        } catch (IOException ioe) {
1657          closeInputStream(RunnerState.DISCONNECTED);
1658        }
1659      }
1660
1661      readBuffer = b;
1662      readOffset = off;
1663      readLength = len;
1664
1665      int count = -1;
1666      count = this.run();
1667      if (count >= 0) {
1668        statistics.incrementBytesRead(count);
1669        pos += count;
1670      } else if (pos < fileLength) {
1671        throw new EOFException(
1672                  "Premature EOF: pos=" + pos + " < filelength=" + fileLength);
1673      }
1674      return count;
1675    }
1676
1677    void seek(long newPos) throws IOException {
1678      if (pos != newPos) {
1679        pos = newPos;
1680        closeInputStream(RunnerState.SEEK);
1681      }
1682    }
1683
1684    public void close() throws IOException {
1685      closeInputStream(RunnerState.CLOSED);
1686    }
1687
1688    /* The following methods are overriding AbstractRunner methods,
1689     * to be called within the retry policy context by runWithRetry.
1690     */
1691
1692    @Override
1693    protected URL getUrl() throws IOException {
1694      // This method is called every time either a read is executed.
1695      // The check for connection == null is to ensure that a new URL is only
1696      // created upon a new connection and not for every read.
1697      if (cachedConnection == null) {
1698        // Update URL with current offset. BufferSize doesn't change, but it
1699        // still must be included when creating the new URL.
1700        updateURLParameters(new BufferSizeParam(bufferSize),
1701            new OffsetParam(pos));
1702        originalUrl = super.getUrl();
1703      }
1704      return originalUrl;
1705    }
1706
1707    /* Only make the connection if it is not already open. Don't cache the
1708     * connection here. After this method is called, runWithRetry will call
1709     * validateResponse, and then call the below ReadRunner#getResponse. If
1710     * the code path makes it that far, then we can cache the connection.
1711     */
1712    @Override
1713    protected HttpURLConnection connect(URL url) throws IOException {
1714      HttpURLConnection conn = cachedConnection;
1715      if (conn == null) {
1716        try {
1717          conn = super.connect(url);
1718        } catch (IOException e) {
1719          closeInputStream(RunnerState.DISCONNECTED);
1720          throw e;
1721        }
1722      }
1723      return conn;
1724    }
1725
1726    /*
1727     * This method is used to perform reads within the retry policy context.
1728     * This code is relying on runWithRetry to always call the above connect
1729     * method and the verifyResponse method prior to calling getResponse.
1730     */
1731    @Override
1732    Integer getResponse(final HttpURLConnection conn)
1733        throws IOException {
1734      try {
1735        // In the "open-then-read" use case, runWithRetry will have executed
1736        // ReadRunner#connect to make the connection and then executed
1737        // validateResponse to validate the response code. Only then do we want
1738        // to cache the connection.
1739        // In the "read-after-seek" use case, the connection is made and the
1740        // response is validated by the URLRunner. ReadRunner#read then caches
1741        // the connection and the ReadRunner#connect will pass on the cached
1742        // connection
1743        // In either case, stream initialization is done here if necessary.
1744        cachedConnection = conn;
1745        if (in == null) {
1746          in = initializeInputStream(conn);
1747        }
1748
1749        int count = in.read(readBuffer, readOffset, readLength);
1750        if (count < 0 && pos < fileLength) {
1751          throw new EOFException(
1752                  "Premature EOF: pos=" + pos + " < filelength=" + fileLength);
1753        }
1754        return Integer.valueOf(count);
1755      } catch (IOException e) {
1756        String redirectHost = resolvedUrl.getAuthority();
1757        if (excludeDatanodes.getValue() != null) {
1758          excludeDatanodes = new ExcludeDatanodesParam(redirectHost + ","
1759              + excludeDatanodes.getValue());
1760        } else {
1761          excludeDatanodes = new ExcludeDatanodesParam(redirectHost);
1762        }
1763
1764        // If an exception occurs, close the input stream and null it out so
1765        // that if the abstract runner decides to retry, it will reconnect.
1766        closeInputStream(RunnerState.DISCONNECTED);
1767        throw e;
1768      }
1769    }
1770
1771    @VisibleForTesting
1772    InputStream initializeInputStream(HttpURLConnection conn)
1773        throws IOException {
1774      // Cache the resolved URL so that it can be used in the event of
1775      // a future seek operation.
1776      resolvedUrl = removeOffsetParam(conn.getURL());
1777      final String cl = conn.getHeaderField(HttpHeaders.CONTENT_LENGTH);
1778      InputStream inStream = conn.getInputStream();
1779      if (LOG.isDebugEnabled()) {
1780        LOG.debug("open file: " + conn.getURL());
1781      }
1782      if (cl != null) {
1783        long streamLength = Long.parseLong(cl);
1784        fileLength = pos + streamLength;
1785        // Java has a bug with >2GB request streams.  It won't bounds check
1786        // the reads so the transfer blocks until the server times out
1787        inStream = new BoundedInputStream(inStream, streamLength);
1788      } else {
1789        fileLength = getHdfsFileStatus(path).getLen();
1790      }
1791      // Wrapping in BufferedInputStream because it is more performant than
1792      // BoundedInputStream by itself.
1793      runnerState = RunnerState.OPEN;
1794      return new BufferedInputStream(inStream, bufferSize);
1795    }
1796
1797    // Close both the InputStream and the connection.
1798    @VisibleForTesting
1799    void closeInputStream(RunnerState rs) throws IOException {
1800      if (in != null) {
1801        IOUtils.close(cachedConnection);
1802        in = null;
1803      }
1804      cachedConnection = null;
1805      runnerState = rs;
1806    }
1807
1808    /* Getters and Setters */
1809
1810    @VisibleForTesting
1811    protected InputStream getInputStream() {
1812      return in;
1813    }
1814
1815    @VisibleForTesting
1816    protected void setInputStream(InputStream inStream) {
1817      in = inStream;
1818    }
1819
1820    Path getPath() {
1821      return path;
1822    }
1823
1824    int getBufferSize() {
1825      return bufferSize;
1826    }
1827
1828    long getFileLength() {
1829      return fileLength;
1830    }
1831
1832    void setFileLength(long len) {
1833      fileLength = len;
1834    }
1835
1836    long getPos() {
1837      return pos;
1838    }
1839  }
1840}