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