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 */
018package org.apache.hadoop.hdfs.server.namenode;
019
020import java.io.File;
021import java.io.FileInputStream;
022import java.io.FileNotFoundException;
023import java.io.FileOutputStream;
024import java.io.IOException;
025import java.io.InputStream;
026import java.io.OutputStream;
027import java.net.HttpURLConnection;
028import java.net.URISyntaxException;
029import java.net.URL;
030import java.security.DigestInputStream;
031import java.security.MessageDigest;
032import java.util.ArrayList;
033import java.util.List;
034import java.util.Map;
035import java.util.Map.Entry;
036
037import javax.servlet.http.HttpServletRequest;
038import javax.servlet.http.HttpServletResponse;
039
040import org.apache.commons.logging.Log;
041import org.apache.commons.logging.LogFactory;
042import org.apache.hadoop.classification.InterfaceAudience;
043import org.apache.hadoop.conf.Configuration;
044import org.apache.hadoop.fs.FileUtil;
045import org.apache.hadoop.hdfs.DFSConfigKeys;
046import org.apache.hadoop.hdfs.HdfsConfiguration;
047import org.apache.hadoop.hdfs.protocol.HdfsConstants;
048import org.apache.hadoop.hdfs.server.common.Storage;
049import org.apache.hadoop.hdfs.server.common.Storage.StorageDirectory;
050import org.apache.hadoop.hdfs.server.common.StorageErrorReporter;
051import org.apache.hadoop.hdfs.server.namenode.NNStorage.NameNodeDirType;
052import org.apache.hadoop.hdfs.server.namenode.NNStorage.NameNodeFile;
053import org.apache.hadoop.hdfs.server.protocol.RemoteEditLog;
054import org.apache.hadoop.hdfs.util.Canceler;
055import org.apache.hadoop.hdfs.util.DataTransferThrottler;
056import org.apache.hadoop.hdfs.web.URLConnectionFactory;
057import org.apache.hadoop.io.IOUtils;
058import org.apache.hadoop.io.MD5Hash;
059import org.apache.hadoop.security.UserGroupInformation;
060import org.apache.hadoop.security.authentication.client.AuthenticationException;
061import org.apache.hadoop.util.Time;
062import org.apache.http.client.utils.URIBuilder;
063
064import com.google.common.annotations.VisibleForTesting;
065import com.google.common.collect.Lists;
066
067
068/**
069 * This class provides fetching a specified file from the NameNode.
070 */
071@InterfaceAudience.Private
072public class TransferFsImage {
073  
074  public final static String CONTENT_LENGTH = "Content-Length";
075  public final static String FILE_LENGTH = "File-Length";
076  public final static String MD5_HEADER = "X-MD5-Digest";
077
078  private final static String CONTENT_TYPE = "Content-Type";
079  private final static String CONTENT_TRANSFER_ENCODING = "Content-Transfer-Encoding";
080
081  @VisibleForTesting
082  static int timeout = 0;
083  private static final URLConnectionFactory connectionFactory;
084  private static final boolean isSpnegoEnabled;
085
086  static {
087    Configuration conf = new Configuration();
088    connectionFactory = URLConnectionFactory
089        .newDefaultURLConnectionFactory(conf);
090    isSpnegoEnabled = UserGroupInformation.isSecurityEnabled();
091  }
092
093  private static final Log LOG = LogFactory.getLog(TransferFsImage.class);
094  
095  public static void downloadMostRecentImageToDirectory(URL infoServer,
096      File dir) throws IOException {
097    String fileId = ImageServlet.getParamStringForMostRecentImage();
098    getFileClient(infoServer, fileId, Lists.newArrayList(dir),
099        null, false);
100  }
101
102  public static MD5Hash downloadImageToStorage(URL fsName, long imageTxId,
103      Storage dstStorage, boolean needDigest) throws IOException {
104    String fileid = ImageServlet.getParamStringForImage(null,
105        imageTxId, dstStorage);
106    String fileName = NNStorage.getCheckpointImageFileName(imageTxId);
107    
108    List<File> dstFiles = dstStorage.getFiles(
109        NameNodeDirType.IMAGE, fileName);
110    if (dstFiles.isEmpty()) {
111      throw new IOException("No targets in destination storage!");
112    }
113    
114    MD5Hash hash = getFileClient(fsName, fileid, dstFiles, dstStorage, needDigest);
115    LOG.info("Downloaded file " + dstFiles.get(0).getName() + " size " +
116        dstFiles.get(0).length() + " bytes.");
117    return hash;
118  }
119
120  static MD5Hash handleUploadImageRequest(HttpServletRequest request,
121      long imageTxId, Storage dstStorage, InputStream stream,
122      long advertisedSize, DataTransferThrottler throttler) throws IOException {
123
124    String fileName = NNStorage.getCheckpointImageFileName(imageTxId);
125
126    List<File> dstFiles = dstStorage.getFiles(NameNodeDirType.IMAGE, fileName);
127    if (dstFiles.isEmpty()) {
128      throw new IOException("No targets in destination storage!");
129    }
130
131    MD5Hash advertisedDigest = parseMD5Header(request);
132    MD5Hash hash = receiveFile(fileName, dstFiles, dstStorage, true,
133        advertisedSize, advertisedDigest, fileName, stream, throttler);
134    LOG.info("Downloaded file " + dstFiles.get(0).getName() + " size "
135        + dstFiles.get(0).length() + " bytes.");
136    return hash;
137  }
138
139  static void downloadEditsToStorage(URL fsName, RemoteEditLog log,
140      NNStorage dstStorage) throws IOException {
141    assert log.getStartTxId() > 0 && log.getEndTxId() > 0 :
142      "bad log: " + log;
143    String fileid = ImageServlet.getParamStringForLog(
144        log, dstStorage);
145    String finalFileName = NNStorage.getFinalizedEditsFileName(
146        log.getStartTxId(), log.getEndTxId());
147
148    List<File> finalFiles = dstStorage.getFiles(NameNodeDirType.EDITS,
149        finalFileName);
150    assert !finalFiles.isEmpty() : "No checkpoint targets.";
151    
152    for (File f : finalFiles) {
153      if (f.exists() && FileUtil.canRead(f)) {
154        LOG.info("Skipping download of remote edit log " +
155            log + " since it already is stored locally at " + f);
156        return;
157      } else if (LOG.isDebugEnabled()) {
158        LOG.debug("Dest file: " + f);
159      }
160    }
161
162    final long milliTime = Time.monotonicNow();
163    String tmpFileName = NNStorage.getTemporaryEditsFileName(
164        log.getStartTxId(), log.getEndTxId(), milliTime);
165    List<File> tmpFiles = dstStorage.getFiles(NameNodeDirType.EDITS,
166        tmpFileName);
167    getFileClient(fsName, fileid, tmpFiles, dstStorage, false);
168    LOG.info("Downloaded file " + tmpFiles.get(0).getName() + " size " +
169        finalFiles.get(0).length() + " bytes.");
170
171    CheckpointFaultInjector.getInstance().beforeEditsRename();
172
173    for (StorageDirectory sd : dstStorage.dirIterable(NameNodeDirType.EDITS)) {
174      File tmpFile = NNStorage.getTemporaryEditsFile(sd,
175          log.getStartTxId(), log.getEndTxId(), milliTime);
176      File finalizedFile = NNStorage.getFinalizedEditsFile(sd,
177          log.getStartTxId(), log.getEndTxId());
178      if (LOG.isDebugEnabled()) {
179        LOG.debug("Renaming " + tmpFile + " to " + finalizedFile);
180      }
181      boolean success = tmpFile.renameTo(finalizedFile);
182      if (!success) {
183        LOG.warn("Unable to rename edits file from " + tmpFile
184            + " to " + finalizedFile);
185      }
186    }
187  }
188 
189  /**
190   * Requests that the NameNode download an image from this node.
191   *
192   * @param fsName the http address for the remote NN
193   * @param conf Configuration
194   * @param storage the storage directory to transfer the image from
195   * @param nnf the NameNodeFile type of the image
196   * @param txid the transaction ID of the image to be uploaded
197   * @throws IOException if there is an I/O error
198   */
199  public static void uploadImageFromStorage(URL fsName, Configuration conf,
200      NNStorage storage, NameNodeFile nnf, long txid) throws IOException {
201    uploadImageFromStorage(fsName, conf, storage, nnf, txid, null);
202  }
203
204  /**
205   * Requests that the NameNode download an image from this node.  Allows for
206   * optional external cancelation.
207   *
208   * @param fsName the http address for the remote NN
209   * @param conf Configuration
210   * @param storage the storage directory to transfer the image from
211   * @param nnf the NameNodeFile type of the image
212   * @param txid the transaction ID of the image to be uploaded
213   * @param canceler optional canceler to check for abort of upload
214   * @throws IOException if there is an I/O error or cancellation
215   */
216  public static void uploadImageFromStorage(URL fsName, Configuration conf,
217      NNStorage storage, NameNodeFile nnf, long txid, Canceler canceler)
218      throws IOException {
219    URL url = new URL(fsName, ImageServlet.PATH_SPEC);
220    long startTime = Time.monotonicNow();
221    try {
222      uploadImage(url, conf, storage, nnf, txid, canceler);
223    } catch (HttpPutFailedException e) {
224      if (e.getResponseCode() == HttpServletResponse.SC_CONFLICT) {
225        // this is OK - this means that a previous attempt to upload
226        // this checkpoint succeeded even though we thought it failed.
227        LOG.info("Image upload with txid " + txid + 
228            " conflicted with a previous image upload to the " +
229            "same NameNode. Continuing...", e);
230        return;
231      } else {
232        throw e;
233      }
234    }
235    double xferSec = Math.max(
236        ((float) (Time.monotonicNow() - startTime)) / 1000.0, 0.001);
237    LOG.info("Uploaded image with txid " + txid + " to namenode at " + fsName
238        + " in " + xferSec + " seconds");
239  }
240
241  /*
242   * Uploads the imagefile using HTTP PUT method
243   */
244  private static void uploadImage(URL url, Configuration conf,
245      NNStorage storage, NameNodeFile nnf, long txId, Canceler canceler)
246      throws IOException {
247
248    File imageFile = storage.findImageFile(nnf, txId);
249    if (imageFile == null) {
250      throw new IOException("Could not find image with txid " + txId);
251    }
252
253    HttpURLConnection connection = null;
254    try {
255      URIBuilder uriBuilder = new URIBuilder(url.toURI());
256
257      // write all params for image upload request as query itself.
258      // Request body contains the image to be uploaded.
259      Map<String, String> params = ImageServlet.getParamsForPutImage(storage,
260          txId, imageFile.length(), nnf);
261      for (Entry<String, String> entry : params.entrySet()) {
262        uriBuilder.addParameter(entry.getKey(), entry.getValue());
263      }
264
265      URL urlWithParams = uriBuilder.build().toURL();
266      connection = (HttpURLConnection) connectionFactory.openConnection(
267          urlWithParams, UserGroupInformation.isSecurityEnabled());
268      // Set the request to PUT
269      connection.setRequestMethod("PUT");
270      connection.setDoOutput(true);
271
272      
273      int chunkSize = conf.getInt(
274          DFSConfigKeys.DFS_IMAGE_TRANSFER_CHUNKSIZE_KEY,
275          DFSConfigKeys.DFS_IMAGE_TRANSFER_CHUNKSIZE_DEFAULT);
276      if (imageFile.length() > chunkSize) {
277        // using chunked streaming mode to support upload of 2GB+ files and to
278        // avoid internal buffering.
279        // this mode should be used only if more than chunkSize data is present
280        // to upload. otherwise upload may not happen sometimes.
281        connection.setChunkedStreamingMode(chunkSize);
282      }
283
284      setTimeout(connection);
285
286      // set headers for verification
287      ImageServlet.setVerificationHeadersForPut(connection, imageFile);
288
289      // Write the file to output stream.
290      writeFileToPutRequest(conf, connection, imageFile, canceler);
291
292      int responseCode = connection.getResponseCode();
293      if (responseCode != HttpURLConnection.HTTP_OK) {
294        throw new HttpPutFailedException(connection.getResponseMessage(),
295            responseCode);
296      }
297    } catch (AuthenticationException e) {
298      throw new IOException(e);
299    } catch (URISyntaxException e) {
300      throw new IOException(e);
301    } finally {
302      if (connection != null) {
303        connection.disconnect();
304      }
305    }
306  }
307
308  private static void writeFileToPutRequest(Configuration conf,
309      HttpURLConnection connection, File imageFile, Canceler canceler)
310      throws FileNotFoundException, IOException {
311    connection.setRequestProperty(CONTENT_TYPE, "application/octet-stream");
312    connection.setRequestProperty(CONTENT_TRANSFER_ENCODING, "binary");
313    OutputStream output = connection.getOutputStream();
314    FileInputStream input = new FileInputStream(imageFile);
315    try {
316      copyFileToStream(output, imageFile, input,
317          ImageServlet.getThrottler(conf), canceler);
318    } finally {
319      IOUtils.closeStream(input);
320      IOUtils.closeStream(output);
321    }
322  }
323
324  /**
325   * A server-side method to respond to a getfile http request
326   * Copies the contents of the local file into the output stream.
327   */
328  public static void copyFileToStream(OutputStream out, File localfile,
329      FileInputStream infile, DataTransferThrottler throttler)
330    throws IOException {
331    copyFileToStream(out, localfile, infile, throttler, null);
332  }
333
334  private static void copyFileToStream(OutputStream out, File localfile,
335      FileInputStream infile, DataTransferThrottler throttler,
336      Canceler canceler) throws IOException {
337    byte buf[] = new byte[HdfsConstants.IO_FILE_BUFFER_SIZE];
338    try {
339      CheckpointFaultInjector.getInstance()
340          .aboutToSendFile(localfile);
341
342      if (CheckpointFaultInjector.getInstance().
343            shouldSendShortFile(localfile)) {
344          // Test sending image shorter than localfile
345          long len = localfile.length();
346          buf = new byte[(int)Math.min(len/2, HdfsConstants.IO_FILE_BUFFER_SIZE)];
347          // This will read at most half of the image
348          // and the rest of the image will be sent over the wire
349          infile.read(buf);
350      }
351      int num = 1;
352      while (num > 0) {
353        if (canceler != null && canceler.isCancelled()) {
354          throw new SaveNamespaceCancelledException(
355            canceler.getCancellationReason());
356        }
357        num = infile.read(buf);
358        if (num <= 0) {
359          break;
360        }
361        if (CheckpointFaultInjector.getInstance()
362              .shouldCorruptAByte(localfile)) {
363          // Simulate a corrupted byte on the wire
364          LOG.warn("SIMULATING A CORRUPT BYTE IN IMAGE TRANSFER!");
365          buf[0]++;
366        }
367        
368        out.write(buf, 0, num);
369        if (throttler != null) {
370          throttler.throttle(num, canceler);
371        }
372      }
373    } finally {
374      if (out != null) {
375        out.close();
376      }
377    }
378  }
379
380  /**
381   * Client-side Method to fetch file from a server
382   * Copies the response from the URL to a list of local files.
383   * @param dstStorage if an error occurs writing to one of the files,
384   *                   this storage object will be notified. 
385   * @Return a digest of the received file if getChecksum is true
386   */
387  static MD5Hash getFileClient(URL infoServer,
388      String queryString, List<File> localPaths,
389      Storage dstStorage, boolean getChecksum) throws IOException {
390    URL url = new URL(infoServer, ImageServlet.PATH_SPEC + "?" + queryString);
391    LOG.info("Opening connection to " + url);
392    return doGetUrl(url, localPaths, dstStorage, getChecksum);
393  }
394  
395  public static MD5Hash doGetUrl(URL url, List<File> localPaths,
396      Storage dstStorage, boolean getChecksum) throws IOException {
397    HttpURLConnection connection;
398    try {
399      connection = (HttpURLConnection)
400        connectionFactory.openConnection(url, isSpnegoEnabled);
401    } catch (AuthenticationException e) {
402      throw new IOException(e);
403    }
404
405    setTimeout(connection);
406
407    if (connection.getResponseCode() != HttpURLConnection.HTTP_OK) {
408      throw new HttpGetFailedException(
409          "Image transfer servlet at " + url +
410          " failed with status code " + connection.getResponseCode() +
411          "\nResponse message:\n" + connection.getResponseMessage(),
412          connection);
413    }
414    
415    long advertisedSize;
416    String contentLength = connection.getHeaderField(CONTENT_LENGTH);
417    if (contentLength != null) {
418      advertisedSize = Long.parseLong(contentLength);
419    } else {
420      throw new IOException(CONTENT_LENGTH + " header is not provided " +
421                            "by the namenode when trying to fetch " + url);
422    }
423    MD5Hash advertisedDigest = parseMD5Header(connection);
424    String fsImageName = connection
425        .getHeaderField(ImageServlet.HADOOP_IMAGE_EDITS_HEADER);
426    InputStream stream = connection.getInputStream();
427
428    return receiveFile(url.toExternalForm(), localPaths, dstStorage,
429        getChecksum, advertisedSize, advertisedDigest, fsImageName, stream,
430        null);
431  }
432
433  private static void setTimeout(HttpURLConnection connection) {
434    if (timeout <= 0) {
435      Configuration conf = new HdfsConfiguration();
436      timeout = conf.getInt(DFSConfigKeys.DFS_IMAGE_TRANSFER_TIMEOUT_KEY,
437          DFSConfigKeys.DFS_IMAGE_TRANSFER_TIMEOUT_DEFAULT);
438      LOG.info("Image Transfer timeout configured to " + timeout
439          + " milliseconds");
440    }
441
442    if (timeout > 0) {
443      connection.setConnectTimeout(timeout);
444      connection.setReadTimeout(timeout);
445    }
446  }
447
448  private static MD5Hash receiveFile(String url, List<File> localPaths,
449      Storage dstStorage, boolean getChecksum, long advertisedSize,
450      MD5Hash advertisedDigest, String fsImageName, InputStream stream,
451      DataTransferThrottler throttler) throws IOException {
452    long startTime = Time.monotonicNow();
453    if (localPaths != null) {
454      // If the local paths refer to directories, use the server-provided header
455      // as the filename within that directory
456      List<File> newLocalPaths = new ArrayList<File>();
457      for (File localPath : localPaths) {
458        if (localPath.isDirectory()) {
459          if (fsImageName == null) {
460            throw new IOException("No filename header provided by server");
461          }
462          newLocalPaths.add(new File(localPath, fsImageName));
463        } else {
464          newLocalPaths.add(localPath);
465        }
466      }
467      localPaths = newLocalPaths;
468    }
469    
470
471    long received = 0;
472    MessageDigest digester = null;
473    if (getChecksum) {
474      digester = MD5Hash.getDigester();
475      stream = new DigestInputStream(stream, digester);
476    }
477    boolean finishedReceiving = false;
478
479    List<FileOutputStream> outputStreams = Lists.newArrayList();
480
481    try {
482      if (localPaths != null) {
483        for (File f : localPaths) {
484          try {
485            if (f.exists()) {
486              LOG.warn("Overwriting existing file " + f
487                  + " with file downloaded from " + url);
488            }
489            outputStreams.add(new FileOutputStream(f));
490          } catch (IOException ioe) {
491            LOG.warn("Unable to download file " + f, ioe);
492            // This will be null if we're downloading the fsimage to a file
493            // outside of an NNStorage directory.
494            if (dstStorage != null &&
495                (dstStorage instanceof StorageErrorReporter)) {
496              ((StorageErrorReporter)dstStorage).reportErrorOnFile(f);
497            }
498          }
499        }
500        
501        if (outputStreams.isEmpty()) {
502          throw new IOException(
503              "Unable to download to any storage directory");
504        }
505      }
506      
507      int num = 1;
508      byte[] buf = new byte[HdfsConstants.IO_FILE_BUFFER_SIZE];
509      while (num > 0) {
510        num = stream.read(buf);
511        if (num > 0) {
512          received += num;
513          for (FileOutputStream fos : outputStreams) {
514            fos.write(buf, 0, num);
515          }
516          if (throttler != null) {
517            throttler.throttle(num);
518          }
519        }
520      }
521      finishedReceiving = true;
522    } finally {
523      stream.close();
524      for (FileOutputStream fos : outputStreams) {
525        fos.getChannel().force(true);
526        fos.close();
527      }
528      if (finishedReceiving && received != advertisedSize) {
529        // only throw this exception if we think we read all of it on our end
530        // -- otherwise a client-side IOException would be masked by this
531        // exception that makes it look like a server-side problem!
532        throw new IOException("File " + url + " received length " + received +
533                              " is not of the advertised size " +
534                              advertisedSize);
535      }
536    }
537    double xferSec = Math.max(
538        ((float)(Time.monotonicNow() - startTime)) / 1000.0, 0.001);
539    long xferKb = received / 1024;
540    LOG.info(String.format("Transfer took %.2fs at %.2f KB/s",
541        xferSec, xferKb / xferSec));
542
543    if (digester != null) {
544      MD5Hash computedDigest = new MD5Hash(digester.digest());
545      
546      if (advertisedDigest != null &&
547          !computedDigest.equals(advertisedDigest)) {
548        throw new IOException("File " + url + " computed digest " +
549            computedDigest + " does not match advertised digest " + 
550            advertisedDigest);
551      }
552      return computedDigest;
553    } else {
554      return null;
555    }    
556  }
557
558  private static MD5Hash parseMD5Header(HttpURLConnection connection) {
559    String header = connection.getHeaderField(MD5_HEADER);
560    return (header != null) ? new MD5Hash(header) : null;
561  }
562
563  private static MD5Hash parseMD5Header(HttpServletRequest request) {
564    String header = request.getHeader(MD5_HEADER);
565    return (header != null) ? new MD5Hash(header) : null;
566  }
567
568  public static class HttpGetFailedException extends IOException {
569    private static final long serialVersionUID = 1L;
570    private final int responseCode;
571
572    HttpGetFailedException(String msg, HttpURLConnection connection) throws IOException {
573      super(msg);
574      this.responseCode = connection.getResponseCode();
575    }
576    
577    public int getResponseCode() {
578      return responseCode;
579    }
580  }
581
582  public static class HttpPutFailedException extends IOException {
583    private static final long serialVersionUID = 1L;
584    private final int responseCode;
585
586    HttpPutFailedException(String msg, int responseCode) throws IOException {
587      super(msg);
588      this.responseCode = responseCode;
589    }
590
591    public int getResponseCode() {
592      return responseCode;
593    }
594  }
595
596}