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.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.URL; 031import java.security.PrivilegedExceptionAction; 032import java.util.ArrayList; 033import java.util.EnumSet; 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.FileStatus; 049import org.apache.hadoop.fs.FileSystem; 050import org.apache.hadoop.fs.MD5MD5CRC32FileChecksum; 051import org.apache.hadoop.fs.Options; 052import org.apache.hadoop.fs.Path; 053import org.apache.hadoop.fs.XAttrCodec; 054import org.apache.hadoop.fs.XAttrSetFlag; 055import org.apache.hadoop.fs.permission.AclEntry; 056import org.apache.hadoop.fs.permission.AclStatus; 057import org.apache.hadoop.fs.permission.FsPermission; 058import org.apache.hadoop.hdfs.DFSConfigKeys; 059import org.apache.hadoop.hdfs.DFSUtil; 060import org.apache.hadoop.hdfs.HAUtil; 061import org.apache.hadoop.hdfs.protocol.HdfsFileStatus; 062import org.apache.hadoop.hdfs.security.token.delegation.DelegationTokenIdentifier; 063import org.apache.hadoop.hdfs.server.namenode.SafeModeException; 064import org.apache.hadoop.hdfs.web.resources.*; 065import org.apache.hadoop.hdfs.web.resources.HttpOpParam.Op; 066import org.apache.hadoop.io.Text; 067import org.apache.hadoop.io.retry.RetryPolicies; 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.token.SecretManager.InvalidToken; 076import org.apache.hadoop.security.token.Token; 077import org.apache.hadoop.security.token.TokenIdentifier; 078import org.apache.hadoop.security.token.TokenSelector; 079import org.apache.hadoop.security.token.delegation.AbstractDelegationTokenSelector; 080import org.apache.hadoop.util.Progressable; 081import org.mortbay.util.ajax.JSON; 082 083import com.google.common.annotations.VisibleForTesting; 084import com.google.common.base.Charsets; 085import com.google.common.base.Preconditions; 086import com.google.common.collect.Lists; 087 088/** A FileSystem for HDFS over the web. */ 089public class WebHdfsFileSystem extends FileSystem 090 implements DelegationTokenRenewer.Renewable, TokenAspect.TokenManagementDelegator { 091 public static final Log LOG = LogFactory.getLog(WebHdfsFileSystem.class); 092 /** File System URI: {SCHEME}://namenode:port/path/to/file */ 093 public static final String SCHEME = "webhdfs"; 094 /** WebHdfs version. */ 095 public static final int VERSION = 1; 096 /** Http URI: http://namenode:port/{PATH_PREFIX}/path/to/file */ 097 public static final String PATH_PREFIX = "/" + SCHEME + "/v" + VERSION; 098 099 /** Default connection factory may be overridden in tests to use smaller timeout values */ 100 protected URLConnectionFactory connectionFactory; 101 102 /** Delegation token kind */ 103 public static final Text TOKEN_KIND = new Text("WEBHDFS delegation"); 104 private boolean canRefreshDelegationToken; 105 106 private UserGroupInformation ugi; 107 private URI uri; 108 private Token<?> delegationToken; 109 protected Text tokenServiceName; 110 private RetryPolicy retryPolicy = null; 111 private Path workingDir; 112 private InetSocketAddress nnAddrs[]; 113 private int currentNNAddrIndex; 114 115 /** 116 * Return the protocol scheme for the FileSystem. 117 * <p/> 118 * 119 * @return <code>webhdfs</code> 120 */ 121 @Override 122 public String getScheme() { 123 return SCHEME; 124 } 125 126 /** 127 * return the underlying transport protocol (http / https). 128 */ 129 protected String getTransportScheme() { 130 return "http"; 131 } 132 133 protected Text getTokenKind() { 134 return TOKEN_KIND; 135 } 136 137 @Override 138 public synchronized void initialize(URI uri, Configuration conf 139 ) throws IOException { 140 super.initialize(uri, conf); 141 setConf(conf); 142 /** set user pattern based on configuration file */ 143 UserParam.setUserPattern(conf.get( 144 DFSConfigKeys.DFS_WEBHDFS_USER_PATTERN_KEY, 145 DFSConfigKeys.DFS_WEBHDFS_USER_PATTERN_DEFAULT)); 146 147 connectionFactory = URLConnectionFactory 148 .newDefaultURLConnectionFactory(conf); 149 150 ugi = UserGroupInformation.getCurrentUser(); 151 this.uri = URI.create(uri.getScheme() + "://" + uri.getAuthority()); 152 this.nnAddrs = resolveNNAddr(); 153 154 boolean isHA = HAUtil.isClientFailoverConfigured(conf, this.uri); 155 boolean isLogicalUri = isHA && HAUtil.isLogicalUri(conf, this.uri); 156 // In non-HA or non-logical URI case, the code needs to call 157 // getCanonicalUri() in order to handle the case where no port is 158 // specified in the URI 159 this.tokenServiceName = isLogicalUri ? 160 HAUtil.buildTokenServiceForLogicalUri(uri) 161 : SecurityUtil.buildTokenService(getCanonicalUri()); 162 163 if (!isHA) { 164 this.retryPolicy = 165 RetryUtils.getDefaultRetryPolicy( 166 conf, 167 DFSConfigKeys.DFS_HTTP_CLIENT_RETRY_POLICY_ENABLED_KEY, 168 DFSConfigKeys.DFS_HTTP_CLIENT_RETRY_POLICY_ENABLED_DEFAULT, 169 DFSConfigKeys.DFS_HTTP_CLIENT_RETRY_POLICY_SPEC_KEY, 170 DFSConfigKeys.DFS_HTTP_CLIENT_RETRY_POLICY_SPEC_DEFAULT, 171 SafeModeException.class); 172 } else { 173 174 int maxFailoverAttempts = conf.getInt( 175 DFSConfigKeys.DFS_HTTP_CLIENT_FAILOVER_MAX_ATTEMPTS_KEY, 176 DFSConfigKeys.DFS_HTTP_CLIENT_FAILOVER_MAX_ATTEMPTS_DEFAULT); 177 int maxRetryAttempts = conf.getInt( 178 DFSConfigKeys.DFS_HTTP_CLIENT_RETRY_MAX_ATTEMPTS_KEY, 179 DFSConfigKeys.DFS_HTTP_CLIENT_RETRY_MAX_ATTEMPTS_DEFAULT); 180 int failoverSleepBaseMillis = conf.getInt( 181 DFSConfigKeys.DFS_HTTP_CLIENT_FAILOVER_SLEEPTIME_BASE_KEY, 182 DFSConfigKeys.DFS_HTTP_CLIENT_FAILOVER_SLEEPTIME_BASE_DEFAULT); 183 int failoverSleepMaxMillis = conf.getInt( 184 DFSConfigKeys.DFS_HTTP_CLIENT_FAILOVER_SLEEPTIME_MAX_KEY, 185 DFSConfigKeys.DFS_HTTP_CLIENT_FAILOVER_SLEEPTIME_MAX_DEFAULT); 186 187 this.retryPolicy = RetryPolicies 188 .failoverOnNetworkException(RetryPolicies.TRY_ONCE_THEN_FAIL, 189 maxFailoverAttempts, maxRetryAttempts, failoverSleepBaseMillis, 190 failoverSleepMaxMillis); 191 } 192 193 this.workingDir = getHomeDirectory(); 194 this.canRefreshDelegationToken = UserGroupInformation.isSecurityEnabled(); 195 this.delegationToken = null; 196 } 197 198 @Override 199 public URI getCanonicalUri() { 200 return super.getCanonicalUri(); 201 } 202 203 /** Is WebHDFS enabled in conf? */ 204 public static boolean isEnabled(final Configuration conf, final Log log) { 205 final boolean b = conf.getBoolean(DFSConfigKeys.DFS_WEBHDFS_ENABLED_KEY, 206 DFSConfigKeys.DFS_WEBHDFS_ENABLED_DEFAULT); 207 return b; 208 } 209 210 TokenSelector<DelegationTokenIdentifier> tokenSelector = 211 new AbstractDelegationTokenSelector<DelegationTokenIdentifier>(getTokenKind()){}; 212 213 // the first getAuthParams() for a non-token op will either get the 214 // internal token from the ugi or lazy fetch one 215 protected synchronized Token<?> getDelegationToken() throws IOException { 216 if (canRefreshDelegationToken && delegationToken == null) { 217 Token<?> token = tokenSelector.selectToken( 218 new Text(getCanonicalServiceName()), ugi.getTokens()); 219 // ugi tokens are usually indicative of a task which can't 220 // refetch tokens. even if ugi has credentials, don't attempt 221 // to get another token to match hdfs/rpc behavior 222 if (token != null) { 223 LOG.debug("Using UGI token: " + token); 224 canRefreshDelegationToken = false; 225 } else { 226 token = getDelegationToken(null); 227 if (token != null) { 228 LOG.debug("Fetched new token: " + token); 229 } else { // security is disabled 230 canRefreshDelegationToken = false; 231 } 232 } 233 setDelegationToken(token); 234 } 235 return delegationToken; 236 } 237 238 @VisibleForTesting 239 synchronized boolean replaceExpiredDelegationToken() throws IOException { 240 boolean replaced = false; 241 if (canRefreshDelegationToken) { 242 Token<?> token = getDelegationToken(null); 243 LOG.debug("Replaced expired token: " + token); 244 setDelegationToken(token); 245 replaced = (token != null); 246 } 247 return replaced; 248 } 249 250 @Override 251 @VisibleForTesting 252 public int getDefaultPort() { 253 return getConf().getInt(DFSConfigKeys.DFS_NAMENODE_HTTP_PORT_KEY, 254 DFSConfigKeys.DFS_NAMENODE_HTTP_PORT_DEFAULT); 255 } 256 257 @Override 258 public URI getUri() { 259 return this.uri; 260 } 261 262 @Override 263 protected URI canonicalizeUri(URI uri) { 264 return NetUtils.getCanonicalUri(uri, getDefaultPort()); 265 } 266 267 /** @return the home directory. */ 268 public static String getHomeDirectoryString(final UserGroupInformation ugi) { 269 return "/user/" + ugi.getShortUserName(); 270 } 271 272 @Override 273 public Path getHomeDirectory() { 274 return makeQualified(new Path(getHomeDirectoryString(ugi))); 275 } 276 277 @Override 278 public synchronized Path getWorkingDirectory() { 279 return workingDir; 280 } 281 282 @Override 283 public synchronized void setWorkingDirectory(final Path dir) { 284 String result = makeAbsolute(dir).toUri().getPath(); 285 if (!DFSUtil.isValidName(result)) { 286 throw new IllegalArgumentException("Invalid DFS directory name " + 287 result); 288 } 289 workingDir = makeAbsolute(dir); 290 } 291 292 private Path makeAbsolute(Path f) { 293 return f.isAbsolute()? f: new Path(workingDir, f); 294 } 295 296 static Map<?, ?> jsonParse(final HttpURLConnection c, final boolean useErrorStream 297 ) throws IOException { 298 if (c.getContentLength() == 0) { 299 return null; 300 } 301 final InputStream in = useErrorStream? c.getErrorStream(): c.getInputStream(); 302 if (in == null) { 303 throw new IOException("The " + (useErrorStream? "error": "input") + " stream is null."); 304 } 305 final String contentType = c.getContentType(); 306 if (contentType != null) { 307 final MediaType parsed = MediaType.valueOf(contentType); 308 if (!MediaType.APPLICATION_JSON_TYPE.isCompatible(parsed)) { 309 throw new IOException("Content-Type \"" + contentType 310 + "\" is incompatible with \"" + MediaType.APPLICATION_JSON 311 + "\" (parsed=\"" + parsed + "\")"); 312 } 313 } 314 return (Map<?, ?>)JSON.parse(new InputStreamReader(in, Charsets.UTF_8)); 315 } 316 317 private static Map<?, ?> validateResponse(final HttpOpParam.Op op, 318 final HttpURLConnection conn, boolean unwrapException) throws IOException { 319 final int code = conn.getResponseCode(); 320 // server is demanding an authentication we don't support 321 if (code == HttpURLConnection.HTTP_UNAUTHORIZED) { 322 // match hdfs/rpc exception 323 throw new AccessControlException(conn.getResponseMessage()); 324 } 325 if (code != op.getExpectedHttpResponseCode()) { 326 final Map<?, ?> m; 327 try { 328 m = jsonParse(conn, true); 329 } catch(Exception e) { 330 throw new IOException("Unexpected HTTP response: code=" + code + " != " 331 + op.getExpectedHttpResponseCode() + ", " + op.toQueryString() 332 + ", message=" + conn.getResponseMessage(), e); 333 } 334 335 if (m == null) { 336 throw new IOException("Unexpected HTTP response: code=" + code + " != " 337 + op.getExpectedHttpResponseCode() + ", " + op.toQueryString() 338 + ", message=" + conn.getResponseMessage()); 339 } else if (m.get(RemoteException.class.getSimpleName()) == null) { 340 return m; 341 } 342 343 IOException re = JsonUtil.toRemoteException(m); 344 // extract UGI-related exceptions and unwrap InvalidToken 345 // the NN mangles these exceptions but the DN does not and may need 346 // to re-fetch a token if either report the token is expired 347 if (re.getMessage().startsWith("Failed to obtain user group information:")) { 348 String[] parts = re.getMessage().split(":\\s+", 3); 349 re = new RemoteException(parts[1], parts[2]); 350 re = ((RemoteException)re).unwrapRemoteException(InvalidToken.class); 351 } 352 throw unwrapException? toIOException(re): re; 353 } 354 return null; 355 } 356 357 /** 358 * Covert an exception to an IOException. 359 * 360 * For a non-IOException, wrap it with IOException. 361 * For a RemoteException, unwrap it. 362 * For an IOException which is not a RemoteException, return it. 363 */ 364 private static IOException toIOException(Exception e) { 365 if (!(e instanceof IOException)) { 366 return new IOException(e); 367 } 368 369 final IOException ioe = (IOException)e; 370 if (!(ioe instanceof RemoteException)) { 371 return ioe; 372 } 373 374 return ((RemoteException)ioe).unwrapRemoteException(); 375 } 376 377 private synchronized InetSocketAddress getCurrentNNAddr() { 378 return nnAddrs[currentNNAddrIndex]; 379 } 380 381 /** 382 * Reset the appropriate state to gracefully fail over to another name node 383 */ 384 private synchronized void resetStateToFailOver() { 385 currentNNAddrIndex = (currentNNAddrIndex + 1) % nnAddrs.length; 386 } 387 388 /** 389 * Return a URL pointing to given path on the namenode. 390 * 391 * @param path to obtain the URL for 392 * @param query string to append to the path 393 * @return namenode URL referring to the given path 394 * @throws IOException on error constructing the URL 395 */ 396 private URL getNamenodeURL(String path, String query) throws IOException { 397 InetSocketAddress nnAddr = getCurrentNNAddr(); 398 final URL url = new URL(getTransportScheme(), nnAddr.getHostName(), 399 nnAddr.getPort(), path + '?' + query); 400 if (LOG.isTraceEnabled()) { 401 LOG.trace("url=" + url); 402 } 403 return url; 404 } 405 406 Param<?,?>[] getAuthParameters(final HttpOpParam.Op op) throws IOException { 407 List<Param<?,?>> authParams = Lists.newArrayList(); 408 // Skip adding delegation token for token operations because these 409 // operations require authentication. 410 Token<?> token = null; 411 if (!op.getRequireAuth()) { 412 token = getDelegationToken(); 413 } 414 if (token != null) { 415 authParams.add(new DelegationParam(token.encodeToUrlString())); 416 } else { 417 UserGroupInformation userUgi = ugi; 418 UserGroupInformation realUgi = userUgi.getRealUser(); 419 if (realUgi != null) { // proxy user 420 authParams.add(new DoAsParam(userUgi.getShortUserName())); 421 userUgi = realUgi; 422 } 423 authParams.add(new UserParam(userUgi.getShortUserName())); 424 } 425 return authParams.toArray(new Param<?,?>[0]); 426 } 427 428 URL toUrl(final HttpOpParam.Op op, final Path fspath, 429 final Param<?,?>... parameters) throws IOException { 430 //initialize URI path and query 431 final String path = PATH_PREFIX 432 + (fspath == null? "/": makeQualified(fspath).toUri().getRawPath()); 433 final String query = op.toQueryString() 434 + Param.toSortedString("&", getAuthParameters(op)) 435 + Param.toSortedString("&", parameters); 436 final URL url = getNamenodeURL(path, query); 437 if (LOG.isTraceEnabled()) { 438 LOG.trace("url=" + url); 439 } 440 return url; 441 } 442 443 /** 444 * This class is for initialing a HTTP connection, connecting to server, 445 * obtaining a response, and also handling retry on failures. 446 */ 447 abstract class AbstractRunner<T> { 448 abstract protected URL getUrl() throws IOException; 449 450 protected final HttpOpParam.Op op; 451 private final boolean redirected; 452 453 private boolean checkRetry; 454 455 protected AbstractRunner(final HttpOpParam.Op op, boolean redirected) { 456 this.op = op; 457 this.redirected = redirected; 458 } 459 460 T run() throws IOException { 461 UserGroupInformation connectUgi = ugi.getRealUser(); 462 if (connectUgi == null) { 463 connectUgi = ugi; 464 } 465 if (op.getRequireAuth()) { 466 connectUgi.checkTGTAndReloginFromKeytab(); 467 } 468 try { 469 // the entire lifecycle of the connection must be run inside the 470 // doAs to ensure authentication is performed correctly 471 return connectUgi.doAs( 472 new PrivilegedExceptionAction<T>() { 473 @Override 474 public T run() throws IOException { 475 return runWithRetry(); 476 } 477 }); 478 } catch (InterruptedException e) { 479 throw new IOException(e); 480 } 481 } 482 483 /** 484 * Two-step requests redirected to a DN 485 * 486 * Create/Append: 487 * Step 1) Submit a Http request with neither auto-redirect nor data. 488 * Step 2) Submit another Http request with the URL from the Location header with data. 489 * 490 * The reason of having two-step create/append is for preventing clients to 491 * send out the data before the redirect. This issue is addressed by the 492 * "Expect: 100-continue" header in HTTP/1.1; see RFC 2616, Section 8.2.3. 493 * Unfortunately, there are software library bugs (e.g. Jetty 6 http server 494 * and Java 6 http client), which do not correctly implement "Expect: 495 * 100-continue". The two-step create/append is a temporary workaround for 496 * the software library bugs. 497 * 498 * Open/Checksum 499 * Also implements two-step connects for other operations redirected to 500 * a DN such as open and checksum 501 */ 502 private HttpURLConnection connect(URL url) throws IOException { 503 // resolve redirects for a DN operation unless already resolved 504 if (op.getRedirect() && !redirected) { 505 final HttpOpParam.Op redirectOp = 506 HttpOpParam.TemporaryRedirectOp.valueOf(op); 507 final HttpURLConnection conn = connect(redirectOp, url); 508 // application level proxy like httpfs might not issue a redirect 509 if (conn.getResponseCode() == op.getExpectedHttpResponseCode()) { 510 return conn; 511 } 512 try { 513 validateResponse(redirectOp, conn, false); 514 url = new URL(conn.getHeaderField("Location")); 515 } finally { 516 conn.disconnect(); 517 } 518 } 519 return connect(op, url); 520 } 521 522 private HttpURLConnection connect(final HttpOpParam.Op op, final URL url) 523 throws IOException { 524 final HttpURLConnection conn = 525 (HttpURLConnection)connectionFactory.openConnection(url); 526 final boolean doOutput = op.getDoOutput(); 527 conn.setRequestMethod(op.getType().toString()); 528 conn.setInstanceFollowRedirects(false); 529 switch (op.getType()) { 530 // if not sending a message body for a POST or PUT operation, need 531 // to ensure the server/proxy knows this 532 case POST: 533 case PUT: { 534 conn.setDoOutput(true); 535 if (!doOutput) { 536 // explicitly setting content-length to 0 won't do spnego!! 537 // opening and closing the stream will send "Content-Length: 0" 538 conn.getOutputStream().close(); 539 } else { 540 conn.setRequestProperty("Content-Type", 541 MediaType.APPLICATION_OCTET_STREAM); 542 conn.setChunkedStreamingMode(32 << 10); //32kB-chunk 543 } 544 break; 545 } 546 default: { 547 conn.setDoOutput(doOutput); 548 break; 549 } 550 } 551 conn.connect(); 552 return conn; 553 } 554 555 private T runWithRetry() throws IOException { 556 /** 557 * Do the real work. 558 * 559 * There are three cases that the code inside the loop can throw an 560 * IOException: 561 * 562 * <ul> 563 * <li>The connection has failed (e.g., ConnectException, 564 * @see FailoverOnNetworkExceptionRetry for more details)</li> 565 * <li>The namenode enters the standby state (i.e., StandbyException).</li> 566 * <li>The server returns errors for the command (i.e., RemoteException)</li> 567 * </ul> 568 * 569 * The call to shouldRetry() will conduct the retry policy. The policy 570 * examines the exception and swallows it if it decides to rerun the work. 571 */ 572 for(int retry = 0; ; retry++) { 573 checkRetry = !redirected; 574 final URL url = getUrl(); 575 try { 576 final HttpURLConnection conn = connect(url); 577 // output streams will validate on close 578 if (!op.getDoOutput()) { 579 validateResponse(op, conn, false); 580 } 581 return getResponse(conn); 582 } catch (AccessControlException ace) { 583 // no retries for auth failures 584 throw ace; 585 } catch (InvalidToken it) { 586 // try to replace the expired token with a new one. the attempt 587 // to acquire a new token must be outside this operation's retry 588 // so if it fails after its own retries, this operation fails too. 589 if (op.getRequireAuth() || !replaceExpiredDelegationToken()) { 590 throw it; 591 } 592 } catch (IOException ioe) { 593 shouldRetry(ioe, retry); 594 } 595 } 596 } 597 598 private void shouldRetry(final IOException ioe, final int retry 599 ) throws IOException { 600 InetSocketAddress nnAddr = getCurrentNNAddr(); 601 if (checkRetry) { 602 try { 603 final RetryPolicy.RetryAction a = retryPolicy.shouldRetry( 604 ioe, retry, 0, true); 605 606 boolean isRetry = a.action == RetryPolicy.RetryAction.RetryDecision.RETRY; 607 boolean isFailoverAndRetry = 608 a.action == RetryPolicy.RetryAction.RetryDecision.FAILOVER_AND_RETRY; 609 610 if (isRetry || isFailoverAndRetry) { 611 LOG.info("Retrying connect to namenode: " + nnAddr 612 + ". Already tried " + retry + " time(s); retry policy is " 613 + retryPolicy + ", delay " + a.delayMillis + "ms."); 614 615 if (isFailoverAndRetry) { 616 resetStateToFailOver(); 617 } 618 619 Thread.sleep(a.delayMillis); 620 return; 621 } 622 } catch(Exception e) { 623 LOG.warn("Original exception is ", ioe); 624 throw toIOException(e); 625 } 626 } 627 throw toIOException(ioe); 628 } 629 630 abstract T getResponse(HttpURLConnection conn) throws IOException; 631 } 632 633 /** 634 * Abstract base class to handle path-based operations with params 635 */ 636 abstract class AbstractFsPathRunner<T> extends AbstractRunner<T> { 637 private final Path fspath; 638 private final Param<?,?>[] parameters; 639 640 AbstractFsPathRunner(final HttpOpParam.Op op, final Path fspath, 641 Param<?,?>... parameters) { 642 super(op, false); 643 this.fspath = fspath; 644 this.parameters = parameters; 645 } 646 647 AbstractFsPathRunner(final HttpOpParam.Op op, Param<?,?>[] parameters, 648 final Path fspath) { 649 super(op, false); 650 this.fspath = fspath; 651 this.parameters = parameters; 652 } 653 654 @Override 655 protected URL getUrl() throws IOException { 656 return toUrl(op, fspath, parameters); 657 } 658 } 659 660 /** 661 * Default path-based implementation expects no json response 662 */ 663 class FsPathRunner extends AbstractFsPathRunner<Void> { 664 FsPathRunner(Op op, Path fspath, Param<?,?>... parameters) { 665 super(op, fspath, parameters); 666 } 667 668 @Override 669 Void getResponse(HttpURLConnection conn) throws IOException { 670 return null; 671 } 672 } 673 674 /** 675 * Handle path-based operations with a json response 676 */ 677 abstract class FsPathResponseRunner<T> extends AbstractFsPathRunner<T> { 678 FsPathResponseRunner(final HttpOpParam.Op op, final Path fspath, 679 Param<?,?>... parameters) { 680 super(op, fspath, parameters); 681 } 682 683 FsPathResponseRunner(final HttpOpParam.Op op, Param<?,?>[] parameters, 684 final Path fspath) { 685 super(op, parameters, fspath); 686 } 687 688 @Override 689 final T getResponse(HttpURLConnection conn) throws IOException { 690 try { 691 final Map<?,?> json = jsonParse(conn, false); 692 if (json == null) { 693 // match exception class thrown by parser 694 throw new IllegalStateException("Missing response"); 695 } 696 return decodeResponse(json); 697 } catch (IOException ioe) { 698 throw ioe; 699 } catch (Exception e) { // catch json parser errors 700 final IOException ioe = 701 new IOException("Response decoding failure: "+e.toString(), e); 702 if (LOG.isDebugEnabled()) { 703 LOG.debug(ioe); 704 } 705 throw ioe; 706 } finally { 707 conn.disconnect(); 708 } 709 } 710 711 abstract T decodeResponse(Map<?,?> json) throws IOException; 712 } 713 714 /** 715 * Handle path-based operations with json boolean response 716 */ 717 class FsPathBooleanRunner extends FsPathResponseRunner<Boolean> { 718 FsPathBooleanRunner(Op op, Path fspath, Param<?,?>... parameters) { 719 super(op, fspath, parameters); 720 } 721 722 @Override 723 Boolean decodeResponse(Map<?,?> json) throws IOException { 724 return (Boolean)json.get("boolean"); 725 } 726 } 727 728 /** 729 * Handle create/append output streams 730 */ 731 class FsPathOutputStreamRunner extends AbstractFsPathRunner<FSDataOutputStream> { 732 private final int bufferSize; 733 734 FsPathOutputStreamRunner(Op op, Path fspath, int bufferSize, 735 Param<?,?>... parameters) { 736 super(op, fspath, parameters); 737 this.bufferSize = bufferSize; 738 } 739 740 @Override 741 FSDataOutputStream getResponse(final HttpURLConnection conn) 742 throws IOException { 743 return new FSDataOutputStream(new BufferedOutputStream( 744 conn.getOutputStream(), bufferSize), statistics) { 745 @Override 746 public void close() throws IOException { 747 try { 748 super.close(); 749 } finally { 750 try { 751 validateResponse(op, conn, true); 752 } finally { 753 conn.disconnect(); 754 } 755 } 756 } 757 }; 758 } 759 } 760 761 class FsPathConnectionRunner extends AbstractFsPathRunner<HttpURLConnection> { 762 FsPathConnectionRunner(Op op, Path fspath, Param<?,?>... parameters) { 763 super(op, fspath, parameters); 764 } 765 @Override 766 HttpURLConnection getResponse(final HttpURLConnection conn) 767 throws IOException { 768 return conn; 769 } 770 } 771 772 /** 773 * Used by open() which tracks the resolved url itself 774 */ 775 final class URLRunner extends AbstractRunner<HttpURLConnection> { 776 private final URL url; 777 @Override 778 protected URL getUrl() { 779 return url; 780 } 781 782 protected URLRunner(final HttpOpParam.Op op, final URL url, boolean redirected) { 783 super(op, redirected); 784 this.url = url; 785 } 786 787 @Override 788 HttpURLConnection getResponse(HttpURLConnection conn) throws IOException { 789 return conn; 790 } 791 } 792 793 private FsPermission applyUMask(FsPermission permission) { 794 if (permission == null) { 795 permission = FsPermission.getDefault(); 796 } 797 return permission.applyUMask(FsPermission.getUMask(getConf())); 798 } 799 800 private HdfsFileStatus getHdfsFileStatus(Path f) throws IOException { 801 final HttpOpParam.Op op = GetOpParam.Op.GETFILESTATUS; 802 HdfsFileStatus status = new FsPathResponseRunner<HdfsFileStatus>(op, f) { 803 @Override 804 HdfsFileStatus decodeResponse(Map<?,?> json) { 805 return JsonUtil.toFileStatus(json, true); 806 } 807 }.run(); 808 if (status == null) { 809 throw new FileNotFoundException("File does not exist: " + f); 810 } 811 return status; 812 } 813 814 @Override 815 public FileStatus getFileStatus(Path f) throws IOException { 816 statistics.incrementReadOps(1); 817 return makeQualified(getHdfsFileStatus(f), f); 818 } 819 820 private FileStatus makeQualified(HdfsFileStatus f, Path parent) { 821 return new FileStatus(f.getLen(), f.isDir(), f.getReplication(), 822 f.getBlockSize(), f.getModificationTime(), f.getAccessTime(), 823 f.getPermission(), f.getOwner(), f.getGroup(), 824 f.isSymlink() ? new Path(f.getSymlink()) : null, 825 f.getFullPath(parent).makeQualified(getUri(), getWorkingDirectory())); 826 } 827 828 @Override 829 public AclStatus getAclStatus(Path f) throws IOException { 830 final HttpOpParam.Op op = GetOpParam.Op.GETACLSTATUS; 831 AclStatus status = new FsPathResponseRunner<AclStatus>(op, f) { 832 @Override 833 AclStatus decodeResponse(Map<?,?> json) { 834 return JsonUtil.toAclStatus(json); 835 } 836 }.run(); 837 if (status == null) { 838 throw new FileNotFoundException("File does not exist: " + f); 839 } 840 return status; 841 } 842 843 @Override 844 public boolean mkdirs(Path f, FsPermission permission) throws IOException { 845 statistics.incrementWriteOps(1); 846 final HttpOpParam.Op op = PutOpParam.Op.MKDIRS; 847 return new FsPathBooleanRunner(op, f, 848 new PermissionParam(applyUMask(permission)) 849 ).run(); 850 } 851 852 /** 853 * Create a symlink pointing to the destination path. 854 * @see org.apache.hadoop.fs.Hdfs#createSymlink(Path, Path, boolean) 855 */ 856 public void createSymlink(Path destination, Path f, boolean createParent 857 ) throws IOException { 858 statistics.incrementWriteOps(1); 859 final HttpOpParam.Op op = PutOpParam.Op.CREATESYMLINK; 860 new FsPathRunner(op, f, 861 new DestinationParam(makeQualified(destination).toUri().getPath()), 862 new CreateParentParam(createParent) 863 ).run(); 864 } 865 866 @Override 867 public boolean rename(final Path src, final Path dst) throws IOException { 868 statistics.incrementWriteOps(1); 869 final HttpOpParam.Op op = PutOpParam.Op.RENAME; 870 return new FsPathBooleanRunner(op, src, 871 new DestinationParam(makeQualified(dst).toUri().getPath()) 872 ).run(); 873 } 874 875 @SuppressWarnings("deprecation") 876 @Override 877 public void rename(final Path src, final Path dst, 878 final Options.Rename... options) throws IOException { 879 statistics.incrementWriteOps(1); 880 final HttpOpParam.Op op = PutOpParam.Op.RENAME; 881 new FsPathRunner(op, src, 882 new DestinationParam(makeQualified(dst).toUri().getPath()), 883 new RenameOptionSetParam(options) 884 ).run(); 885 } 886 887 @Override 888 public void setXAttr(Path p, String name, byte[] value, 889 EnumSet<XAttrSetFlag> flag) throws IOException { 890 statistics.incrementWriteOps(1); 891 final HttpOpParam.Op op = PutOpParam.Op.SETXATTR; 892 if (value != null) { 893 new FsPathRunner(op, p, new XAttrNameParam(name), new XAttrValueParam( 894 XAttrCodec.encodeValue(value, XAttrCodec.HEX)), 895 new XAttrSetFlagParam(flag)).run(); 896 } else { 897 new FsPathRunner(op, p, new XAttrNameParam(name), 898 new XAttrSetFlagParam(flag)).run(); 899 } 900 } 901 902 @Override 903 public byte[] getXAttr(Path p, final String name) throws IOException { 904 final HttpOpParam.Op op = GetOpParam.Op.GETXATTRS; 905 return new FsPathResponseRunner<byte[]>(op, p, new XAttrNameParam(name), 906 new XAttrEncodingParam(XAttrCodec.HEX)) { 907 @Override 908 byte[] decodeResponse(Map<?, ?> json) throws IOException { 909 return JsonUtil.getXAttr(json, name); 910 } 911 }.run(); 912 } 913 914 @Override 915 public Map<String, byte[]> getXAttrs(Path p) throws IOException { 916 final HttpOpParam.Op op = GetOpParam.Op.GETXATTRS; 917 return new FsPathResponseRunner<Map<String, byte[]>>(op, p, 918 new XAttrEncodingParam(XAttrCodec.HEX)) { 919 @Override 920 Map<String, byte[]> decodeResponse(Map<?, ?> json) throws IOException { 921 return JsonUtil.toXAttrs(json); 922 } 923 }.run(); 924 } 925 926 @Override 927 public Map<String, byte[]> getXAttrs(Path p, final List<String> names) 928 throws IOException { 929 Preconditions.checkArgument(names != null && !names.isEmpty(), 930 "XAttr names cannot be null or empty."); 931 Param<?,?>[] parameters = new Param<?,?>[names.size() + 1]; 932 for (int i = 0; i < parameters.length - 1; i++) { 933 parameters[i] = new XAttrNameParam(names.get(i)); 934 } 935 parameters[parameters.length - 1] = new XAttrEncodingParam(XAttrCodec.HEX); 936 937 final HttpOpParam.Op op = GetOpParam.Op.GETXATTRS; 938 return new FsPathResponseRunner<Map<String, byte[]>>(op, parameters, p) { 939 @Override 940 Map<String, byte[]> decodeResponse(Map<?, ?> json) throws IOException { 941 return JsonUtil.toXAttrs(json); 942 } 943 }.run(); 944 } 945 946 @Override 947 public List<String> listXAttrs(Path p) throws IOException { 948 final HttpOpParam.Op op = GetOpParam.Op.LISTXATTRS; 949 return new FsPathResponseRunner<List<String>>(op, p) { 950 @Override 951 List<String> decodeResponse(Map<?, ?> json) throws IOException { 952 return JsonUtil.toXAttrNames(json); 953 } 954 }.run(); 955 } 956 957 @Override 958 public void removeXAttr(Path p, String name) throws IOException { 959 statistics.incrementWriteOps(1); 960 final HttpOpParam.Op op = PutOpParam.Op.REMOVEXATTR; 961 new FsPathRunner(op, p, new XAttrNameParam(name)).run(); 962 } 963 964 @Override 965 public void setOwner(final Path p, final String owner, final String group 966 ) throws IOException { 967 if (owner == null && group == null) { 968 throw new IOException("owner == null && group == null"); 969 } 970 971 statistics.incrementWriteOps(1); 972 final HttpOpParam.Op op = PutOpParam.Op.SETOWNER; 973 new FsPathRunner(op, p, 974 new OwnerParam(owner), new GroupParam(group) 975 ).run(); 976 } 977 978 @Override 979 public void setPermission(final Path p, final FsPermission permission 980 ) throws IOException { 981 statistics.incrementWriteOps(1); 982 final HttpOpParam.Op op = PutOpParam.Op.SETPERMISSION; 983 new FsPathRunner(op, p,new PermissionParam(permission)).run(); 984 } 985 986 @Override 987 public void modifyAclEntries(Path path, List<AclEntry> aclSpec) 988 throws IOException { 989 statistics.incrementWriteOps(1); 990 final HttpOpParam.Op op = PutOpParam.Op.MODIFYACLENTRIES; 991 new FsPathRunner(op, path, new AclPermissionParam(aclSpec)).run(); 992 } 993 994 @Override 995 public void removeAclEntries(Path path, List<AclEntry> aclSpec) 996 throws IOException { 997 statistics.incrementWriteOps(1); 998 final HttpOpParam.Op op = PutOpParam.Op.REMOVEACLENTRIES; 999 new FsPathRunner(op, path, new AclPermissionParam(aclSpec)).run(); 1000 } 1001 1002 @Override 1003 public void removeDefaultAcl(Path path) throws IOException { 1004 statistics.incrementWriteOps(1); 1005 final HttpOpParam.Op op = PutOpParam.Op.REMOVEDEFAULTACL; 1006 new FsPathRunner(op, path).run(); 1007 } 1008 1009 @Override 1010 public void removeAcl(Path path) throws IOException { 1011 statistics.incrementWriteOps(1); 1012 final HttpOpParam.Op op = PutOpParam.Op.REMOVEACL; 1013 new FsPathRunner(op, path).run(); 1014 } 1015 1016 @Override 1017 public void setAcl(final Path p, final List<AclEntry> aclSpec) 1018 throws IOException { 1019 statistics.incrementWriteOps(1); 1020 final HttpOpParam.Op op = PutOpParam.Op.SETACL; 1021 new FsPathRunner(op, p, new AclPermissionParam(aclSpec)).run(); 1022 } 1023 1024 @Override 1025 public Path createSnapshot(final Path path, final String snapshotName) 1026 throws IOException { 1027 statistics.incrementWriteOps(1); 1028 final HttpOpParam.Op op = PutOpParam.Op.CREATESNAPSHOT; 1029 Path spath = new FsPathResponseRunner<Path>(op, path, 1030 new SnapshotNameParam(snapshotName)) { 1031 @Override 1032 Path decodeResponse(Map<?,?> json) { 1033 return new Path((String) json.get(Path.class.getSimpleName())); 1034 } 1035 }.run(); 1036 return spath; 1037 } 1038 1039 @Override 1040 public void deleteSnapshot(final Path path, final String snapshotName) 1041 throws IOException { 1042 statistics.incrementWriteOps(1); 1043 final HttpOpParam.Op op = DeleteOpParam.Op.DELETESNAPSHOT; 1044 new FsPathRunner(op, path, new SnapshotNameParam(snapshotName)).run(); 1045 } 1046 1047 @Override 1048 public void renameSnapshot(final Path path, final String snapshotOldName, 1049 final String snapshotNewName) throws IOException { 1050 statistics.incrementWriteOps(1); 1051 final HttpOpParam.Op op = PutOpParam.Op.RENAMESNAPSHOT; 1052 new FsPathRunner(op, path, new OldSnapshotNameParam(snapshotOldName), 1053 new SnapshotNameParam(snapshotNewName)).run(); 1054 } 1055 1056 @Override 1057 public boolean setReplication(final Path p, final short replication 1058 ) throws IOException { 1059 statistics.incrementWriteOps(1); 1060 final HttpOpParam.Op op = PutOpParam.Op.SETREPLICATION; 1061 return new FsPathBooleanRunner(op, p, 1062 new ReplicationParam(replication) 1063 ).run(); 1064 } 1065 1066 @Override 1067 public void setTimes(final Path p, final long mtime, final long atime 1068 ) throws IOException { 1069 statistics.incrementWriteOps(1); 1070 final HttpOpParam.Op op = PutOpParam.Op.SETTIMES; 1071 new FsPathRunner(op, p, 1072 new ModificationTimeParam(mtime), 1073 new AccessTimeParam(atime) 1074 ).run(); 1075 } 1076 1077 @Override 1078 public long getDefaultBlockSize() { 1079 return getConf().getLongBytes(DFSConfigKeys.DFS_BLOCK_SIZE_KEY, 1080 DFSConfigKeys.DFS_BLOCK_SIZE_DEFAULT); 1081 } 1082 1083 @Override 1084 public short getDefaultReplication() { 1085 return (short)getConf().getInt(DFSConfigKeys.DFS_REPLICATION_KEY, 1086 DFSConfigKeys.DFS_REPLICATION_DEFAULT); 1087 } 1088 1089 @Override 1090 public void concat(final Path trg, final Path [] srcs) throws IOException { 1091 statistics.incrementWriteOps(1); 1092 final HttpOpParam.Op op = PostOpParam.Op.CONCAT; 1093 new FsPathRunner(op, trg, new ConcatSourcesParam(srcs)).run(); 1094 } 1095 1096 @Override 1097 public FSDataOutputStream create(final Path f, final FsPermission permission, 1098 final boolean overwrite, final int bufferSize, final short replication, 1099 final long blockSize, final Progressable progress) throws IOException { 1100 statistics.incrementWriteOps(1); 1101 1102 final HttpOpParam.Op op = PutOpParam.Op.CREATE; 1103 return new FsPathOutputStreamRunner(op, f, bufferSize, 1104 new PermissionParam(applyUMask(permission)), 1105 new OverwriteParam(overwrite), 1106 new BufferSizeParam(bufferSize), 1107 new ReplicationParam(replication), 1108 new BlockSizeParam(blockSize) 1109 ).run(); 1110 } 1111 1112 @Override 1113 public FSDataOutputStream append(final Path f, final int bufferSize, 1114 final Progressable progress) throws IOException { 1115 statistics.incrementWriteOps(1); 1116 1117 final HttpOpParam.Op op = PostOpParam.Op.APPEND; 1118 return new FsPathOutputStreamRunner(op, f, bufferSize, 1119 new BufferSizeParam(bufferSize) 1120 ).run(); 1121 } 1122 1123 @Override 1124 public boolean delete(Path f, boolean recursive) throws IOException { 1125 final HttpOpParam.Op op = DeleteOpParam.Op.DELETE; 1126 return new FsPathBooleanRunner(op, f, 1127 new RecursiveParam(recursive) 1128 ).run(); 1129 } 1130 1131 @Override 1132 public FSDataInputStream open(final Path f, final int buffersize 1133 ) throws IOException { 1134 statistics.incrementReadOps(1); 1135 final HttpOpParam.Op op = GetOpParam.Op.OPEN; 1136 // use a runner so the open can recover from an invalid token 1137 FsPathConnectionRunner runner = 1138 new FsPathConnectionRunner(op, f, new BufferSizeParam(buffersize)); 1139 return new FSDataInputStream(new OffsetUrlInputStream( 1140 new UnresolvedUrlOpener(runner), new OffsetUrlOpener(null))); 1141 } 1142 1143 @Override 1144 public synchronized void close() throws IOException { 1145 try { 1146 if (canRefreshDelegationToken && delegationToken != null) { 1147 cancelDelegationToken(delegationToken); 1148 } 1149 } catch (IOException ioe) { 1150 LOG.debug("Token cancel failed: "+ioe); 1151 } finally { 1152 super.close(); 1153 } 1154 } 1155 1156 // use FsPathConnectionRunner to ensure retries for InvalidTokens 1157 class UnresolvedUrlOpener extends ByteRangeInputStream.URLOpener { 1158 private final FsPathConnectionRunner runner; 1159 UnresolvedUrlOpener(FsPathConnectionRunner runner) { 1160 super(null); 1161 this.runner = runner; 1162 } 1163 1164 @Override 1165 protected HttpURLConnection connect(long offset, boolean resolved) 1166 throws IOException { 1167 assert offset == 0; 1168 HttpURLConnection conn = runner.run(); 1169 setURL(conn.getURL()); 1170 return conn; 1171 } 1172 } 1173 1174 class OffsetUrlOpener extends ByteRangeInputStream.URLOpener { 1175 OffsetUrlOpener(final URL url) { 1176 super(url); 1177 } 1178 1179 /** Setup offset url and connect. */ 1180 @Override 1181 protected HttpURLConnection connect(final long offset, 1182 final boolean resolved) throws IOException { 1183 final URL offsetUrl = offset == 0L? url 1184 : new URL(url + "&" + new OffsetParam(offset)); 1185 return new URLRunner(GetOpParam.Op.OPEN, offsetUrl, resolved).run(); 1186 } 1187 } 1188 1189 private static final String OFFSET_PARAM_PREFIX = OffsetParam.NAME + "="; 1190 1191 /** Remove offset parameter, if there is any, from the url */ 1192 static URL removeOffsetParam(final URL url) throws MalformedURLException { 1193 String query = url.getQuery(); 1194 if (query == null) { 1195 return url; 1196 } 1197 final String lower = query.toLowerCase(); 1198 if (!lower.startsWith(OFFSET_PARAM_PREFIX) 1199 && !lower.contains("&" + OFFSET_PARAM_PREFIX)) { 1200 return url; 1201 } 1202 1203 //rebuild query 1204 StringBuilder b = null; 1205 for(final StringTokenizer st = new StringTokenizer(query, "&"); 1206 st.hasMoreTokens();) { 1207 final String token = st.nextToken(); 1208 if (!token.toLowerCase().startsWith(OFFSET_PARAM_PREFIX)) { 1209 if (b == null) { 1210 b = new StringBuilder("?").append(token); 1211 } else { 1212 b.append('&').append(token); 1213 } 1214 } 1215 } 1216 query = b == null? "": b.toString(); 1217 1218 final String urlStr = url.toString(); 1219 return new URL(urlStr.substring(0, urlStr.indexOf('?')) + query); 1220 } 1221 1222 static class OffsetUrlInputStream extends ByteRangeInputStream { 1223 OffsetUrlInputStream(UnresolvedUrlOpener o, OffsetUrlOpener r) 1224 throws IOException { 1225 super(o, r); 1226 } 1227 1228 /** Remove offset parameter before returning the resolved url. */ 1229 @Override 1230 protected URL getResolvedUrl(final HttpURLConnection connection 1231 ) throws MalformedURLException { 1232 return removeOffsetParam(connection.getURL()); 1233 } 1234 } 1235 1236 @Override 1237 public FileStatus[] listStatus(final Path f) throws IOException { 1238 statistics.incrementReadOps(1); 1239 1240 final HttpOpParam.Op op = GetOpParam.Op.LISTSTATUS; 1241 return new FsPathResponseRunner<FileStatus[]>(op, f) { 1242 @Override 1243 FileStatus[] decodeResponse(Map<?,?> json) { 1244 final Map<?, ?> rootmap = (Map<?, ?>)json.get(FileStatus.class.getSimpleName() + "es"); 1245 final Object[] array = (Object[])rootmap.get(FileStatus.class.getSimpleName()); 1246 1247 //convert FileStatus 1248 final FileStatus[] statuses = new FileStatus[array.length]; 1249 for (int i = 0; i < array.length; i++) { 1250 final Map<?, ?> m = (Map<?, ?>)array[i]; 1251 statuses[i] = makeQualified(JsonUtil.toFileStatus(m, false), f); 1252 } 1253 return statuses; 1254 } 1255 }.run(); 1256 } 1257 1258 @Override 1259 public Token<DelegationTokenIdentifier> getDelegationToken( 1260 final String renewer) throws IOException { 1261 final HttpOpParam.Op op = GetOpParam.Op.GETDELEGATIONTOKEN; 1262 Token<DelegationTokenIdentifier> token = 1263 new FsPathResponseRunner<Token<DelegationTokenIdentifier>>( 1264 op, null, new RenewerParam(renewer)) { 1265 @Override 1266 Token<DelegationTokenIdentifier> decodeResponse(Map<?,?> json) 1267 throws IOException { 1268 return JsonUtil.toDelegationToken(json); 1269 } 1270 }.run(); 1271 token.setService(tokenServiceName); 1272 return token; 1273 } 1274 1275 @Override 1276 public synchronized Token<?> getRenewToken() { 1277 return delegationToken; 1278 } 1279 1280 @Override 1281 public <T extends TokenIdentifier> void setDelegationToken( 1282 final Token<T> token) { 1283 synchronized (this) { 1284 delegationToken = token; 1285 } 1286 } 1287 1288 @Override 1289 public synchronized long renewDelegationToken(final Token<?> token 1290 ) throws IOException { 1291 final HttpOpParam.Op op = PutOpParam.Op.RENEWDELEGATIONTOKEN; 1292 return new FsPathResponseRunner<Long>(op, null, 1293 new TokenArgumentParam(token.encodeToUrlString())) { 1294 @Override 1295 Long decodeResponse(Map<?,?> json) throws IOException { 1296 return (Long) json.get("long"); 1297 } 1298 }.run(); 1299 } 1300 1301 @Override 1302 public synchronized void cancelDelegationToken(final Token<?> token 1303 ) throws IOException { 1304 final HttpOpParam.Op op = PutOpParam.Op.CANCELDELEGATIONTOKEN; 1305 new FsPathRunner(op, null, 1306 new TokenArgumentParam(token.encodeToUrlString()) 1307 ).run(); 1308 } 1309 1310 @Override 1311 public BlockLocation[] getFileBlockLocations(final FileStatus status, 1312 final long offset, final long length) throws IOException { 1313 if (status == null) { 1314 return null; 1315 } 1316 return getFileBlockLocations(status.getPath(), offset, length); 1317 } 1318 1319 @Override 1320 public BlockLocation[] getFileBlockLocations(final Path p, 1321 final long offset, final long length) throws IOException { 1322 statistics.incrementReadOps(1); 1323 1324 final HttpOpParam.Op op = GetOpParam.Op.GET_BLOCK_LOCATIONS; 1325 return new FsPathResponseRunner<BlockLocation[]>(op, p, 1326 new OffsetParam(offset), new LengthParam(length)) { 1327 @Override 1328 BlockLocation[] decodeResponse(Map<?,?> json) throws IOException { 1329 return DFSUtil.locatedBlocks2Locations( 1330 JsonUtil.toLocatedBlocks(json)); 1331 } 1332 }.run(); 1333 } 1334 1335 @Override 1336 public ContentSummary getContentSummary(final Path p) throws IOException { 1337 statistics.incrementReadOps(1); 1338 1339 final HttpOpParam.Op op = GetOpParam.Op.GETCONTENTSUMMARY; 1340 return new FsPathResponseRunner<ContentSummary>(op, p) { 1341 @Override 1342 ContentSummary decodeResponse(Map<?,?> json) { 1343 return JsonUtil.toContentSummary(json); 1344 } 1345 }.run(); 1346 } 1347 1348 @Override 1349 public MD5MD5CRC32FileChecksum getFileChecksum(final Path p 1350 ) throws IOException { 1351 statistics.incrementReadOps(1); 1352 1353 final HttpOpParam.Op op = GetOpParam.Op.GETFILECHECKSUM; 1354 return new FsPathResponseRunner<MD5MD5CRC32FileChecksum>(op, p) { 1355 @Override 1356 MD5MD5CRC32FileChecksum decodeResponse(Map<?,?> json) throws IOException { 1357 return JsonUtil.toMD5MD5CRC32FileChecksum(json); 1358 } 1359 }.run(); 1360 } 1361 1362 /** 1363 * Resolve an HDFS URL into real INetSocketAddress. It works like a DNS 1364 * resolver when the URL points to an non-HA cluster. When the URL points to 1365 * an HA cluster with its logical name, the resolver further resolves the 1366 * logical name(i.e., the authority in the URL) into real namenode addresses. 1367 */ 1368 private InetSocketAddress[] resolveNNAddr() throws IOException { 1369 Configuration conf = getConf(); 1370 final String scheme = uri.getScheme(); 1371 1372 ArrayList<InetSocketAddress> ret = new ArrayList<InetSocketAddress>(); 1373 1374 if (!HAUtil.isLogicalUri(conf, uri)) { 1375 InetSocketAddress addr = NetUtils.createSocketAddr(uri.getAuthority(), 1376 getDefaultPort()); 1377 ret.add(addr); 1378 1379 } else { 1380 Map<String, Map<String, InetSocketAddress>> addresses = DFSUtil 1381 .getHaNnWebHdfsAddresses(conf, scheme); 1382 1383 // Extract the entry corresponding to the logical name. 1384 Map<String, InetSocketAddress> addrs = addresses.get(uri.getHost()); 1385 for (InetSocketAddress addr : addrs.values()) { 1386 ret.add(addr); 1387 } 1388 } 1389 1390 InetSocketAddress[] r = new InetSocketAddress[ret.size()]; 1391 return ret.toArray(r); 1392 } 1393 1394 @Override 1395 public String getCanonicalServiceName() { 1396 return tokenServiceName == null ? super.getCanonicalServiceName() 1397 : tokenServiceName.toString(); 1398 } 1399 1400 @VisibleForTesting 1401 InetSocketAddress[] getResolvedNNAddr() { 1402 return nnAddrs; 1403 } 1404}