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