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;
066import org.mortbay.jetty.EofException;
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(String.format(
295            "Image uploading failed, status: %d, url: %s, message: %s",
296            responseCode, urlWithParams, connection.getResponseMessage()),
297            responseCode);
298      }
299    } catch (AuthenticationException e) {
300      throw new IOException(e);
301    } catch (URISyntaxException e) {
302      throw new IOException(e);
303    } finally {
304      if (connection != null) {
305        connection.disconnect();
306      }
307    }
308  }
309
310  private static void writeFileToPutRequest(Configuration conf,
311      HttpURLConnection connection, File imageFile, Canceler canceler)
312      throws FileNotFoundException, IOException {
313    connection.setRequestProperty(CONTENT_TYPE, "application/octet-stream");
314    connection.setRequestProperty(CONTENT_TRANSFER_ENCODING, "binary");
315    OutputStream output = connection.getOutputStream();
316    FileInputStream input = new FileInputStream(imageFile);
317    try {
318      copyFileToStream(output, imageFile, input,
319          ImageServlet.getThrottler(conf), canceler);
320    } finally {
321      IOUtils.closeStream(input);
322      IOUtils.closeStream(output);
323    }
324  }
325
326  /**
327   * A server-side method to respond to a getfile http request
328   * Copies the contents of the local file into the output stream.
329   */
330  public static void copyFileToStream(OutputStream out, File localfile,
331      FileInputStream infile, DataTransferThrottler throttler)
332    throws IOException {
333    copyFileToStream(out, localfile, infile, throttler, null);
334  }
335
336  private static void copyFileToStream(OutputStream out, File localfile,
337      FileInputStream infile, DataTransferThrottler throttler,
338      Canceler canceler) throws IOException {
339    byte buf[] = new byte[HdfsConstants.IO_FILE_BUFFER_SIZE];
340    try {
341      CheckpointFaultInjector.getInstance()
342          .aboutToSendFile(localfile);
343
344      if (CheckpointFaultInjector.getInstance().
345            shouldSendShortFile(localfile)) {
346          // Test sending image shorter than localfile
347          long len = localfile.length();
348          buf = new byte[(int)Math.min(len/2, HdfsConstants.IO_FILE_BUFFER_SIZE)];
349          // This will read at most half of the image
350          // and the rest of the image will be sent over the wire
351          infile.read(buf);
352      }
353      int num = 1;
354      while (num > 0) {
355        if (canceler != null && canceler.isCancelled()) {
356          throw new SaveNamespaceCancelledException(
357            canceler.getCancellationReason());
358        }
359        num = infile.read(buf);
360        if (num <= 0) {
361          break;
362        }
363        if (CheckpointFaultInjector.getInstance()
364              .shouldCorruptAByte(localfile)) {
365          // Simulate a corrupted byte on the wire
366          LOG.warn("SIMULATING A CORRUPT BYTE IN IMAGE TRANSFER!");
367          buf[0]++;
368        }
369        
370        out.write(buf, 0, num);
371        if (throttler != null) {
372          throttler.throttle(num, canceler);
373        }
374      }
375    } catch (EofException e) {
376      LOG.info("Connection closed by client");
377      out = null; // so we don't close in the finally
378    } finally {
379      if (out != null) {
380        out.close();
381      }
382    }
383  }
384
385  /**
386   * Client-side Method to fetch file from a server
387   * Copies the response from the URL to a list of local files.
388   * @param dstStorage if an error occurs writing to one of the files,
389   *                   this storage object will be notified. 
390   * @Return a digest of the received file if getChecksum is true
391   */
392  static MD5Hash getFileClient(URL infoServer,
393      String queryString, List<File> localPaths,
394      Storage dstStorage, boolean getChecksum) throws IOException {
395    URL url = new URL(infoServer, ImageServlet.PATH_SPEC + "?" + queryString);
396    LOG.info("Opening connection to " + url);
397    return doGetUrl(url, localPaths, dstStorage, getChecksum);
398  }
399  
400  public static MD5Hash doGetUrl(URL url, List<File> localPaths,
401      Storage dstStorage, boolean getChecksum) throws IOException {
402    HttpURLConnection connection;
403    try {
404      connection = (HttpURLConnection)
405        connectionFactory.openConnection(url, isSpnegoEnabled);
406    } catch (AuthenticationException e) {
407      throw new IOException(e);
408    }
409
410    setTimeout(connection);
411
412    if (connection.getResponseCode() != HttpURLConnection.HTTP_OK) {
413      throw new HttpGetFailedException(
414          "Image transfer servlet at " + url +
415          " failed with status code " + connection.getResponseCode() +
416          "\nResponse message:\n" + connection.getResponseMessage(),
417          connection);
418    }
419    
420    long advertisedSize;
421    String contentLength = connection.getHeaderField(CONTENT_LENGTH);
422    if (contentLength != null) {
423      advertisedSize = Long.parseLong(contentLength);
424    } else {
425      throw new IOException(CONTENT_LENGTH + " header is not provided " +
426                            "by the namenode when trying to fetch " + url);
427    }
428    MD5Hash advertisedDigest = parseMD5Header(connection);
429    String fsImageName = connection
430        .getHeaderField(ImageServlet.HADOOP_IMAGE_EDITS_HEADER);
431    InputStream stream = connection.getInputStream();
432
433    return receiveFile(url.toExternalForm(), localPaths, dstStorage,
434        getChecksum, advertisedSize, advertisedDigest, fsImageName, stream,
435        null);
436  }
437
438  private static void setTimeout(HttpURLConnection connection) {
439    if (timeout <= 0) {
440      Configuration conf = new HdfsConfiguration();
441      timeout = conf.getInt(DFSConfigKeys.DFS_IMAGE_TRANSFER_TIMEOUT_KEY,
442          DFSConfigKeys.DFS_IMAGE_TRANSFER_TIMEOUT_DEFAULT);
443      LOG.info("Image Transfer timeout configured to " + timeout
444          + " milliseconds");
445    }
446
447    if (timeout > 0) {
448      connection.setConnectTimeout(timeout);
449      connection.setReadTimeout(timeout);
450    }
451  }
452
453  private static MD5Hash receiveFile(String url, List<File> localPaths,
454      Storage dstStorage, boolean getChecksum, long advertisedSize,
455      MD5Hash advertisedDigest, String fsImageName, InputStream stream,
456      DataTransferThrottler throttler) throws IOException {
457    long startTime = Time.monotonicNow();
458    if (localPaths != null) {
459      // If the local paths refer to directories, use the server-provided header
460      // as the filename within that directory
461      List<File> newLocalPaths = new ArrayList<File>();
462      for (File localPath : localPaths) {
463        if (localPath.isDirectory()) {
464          if (fsImageName == null) {
465            throw new IOException("No filename header provided by server");
466          }
467          newLocalPaths.add(new File(localPath, fsImageName));
468        } else {
469          newLocalPaths.add(localPath);
470        }
471      }
472      localPaths = newLocalPaths;
473    }
474    
475
476    long received = 0;
477    MessageDigest digester = null;
478    if (getChecksum) {
479      digester = MD5Hash.getDigester();
480      stream = new DigestInputStream(stream, digester);
481    }
482    boolean finishedReceiving = false;
483
484    List<FileOutputStream> outputStreams = Lists.newArrayList();
485
486    try {
487      if (localPaths != null) {
488        for (File f : localPaths) {
489          try {
490            if (f.exists()) {
491              LOG.warn("Overwriting existing file " + f
492                  + " with file downloaded from " + url);
493            }
494            outputStreams.add(new FileOutputStream(f));
495          } catch (IOException ioe) {
496            LOG.warn("Unable to download file " + f, ioe);
497            // This will be null if we're downloading the fsimage to a file
498            // outside of an NNStorage directory.
499            if (dstStorage != null &&
500                (dstStorage instanceof StorageErrorReporter)) {
501              ((StorageErrorReporter)dstStorage).reportErrorOnFile(f);
502            }
503          }
504        }
505        
506        if (outputStreams.isEmpty()) {
507          throw new IOException(
508              "Unable to download to any storage directory");
509        }
510      }
511      
512      int num = 1;
513      byte[] buf = new byte[HdfsConstants.IO_FILE_BUFFER_SIZE];
514      while (num > 0) {
515        num = stream.read(buf);
516        if (num > 0) {
517          received += num;
518          for (FileOutputStream fos : outputStreams) {
519            fos.write(buf, 0, num);
520          }
521          if (throttler != null) {
522            throttler.throttle(num);
523          }
524        }
525      }
526      finishedReceiving = true;
527    } finally {
528      stream.close();
529      for (FileOutputStream fos : outputStreams) {
530        fos.getChannel().force(true);
531        fos.close();
532      }
533
534      // Something went wrong and did not finish reading.
535      // Remove the temporary files.
536      if (!finishedReceiving) {
537        deleteTmpFiles(localPaths);
538      }
539
540      if (finishedReceiving && received != advertisedSize) {
541        // only throw this exception if we think we read all of it on our end
542        // -- otherwise a client-side IOException would be masked by this
543        // exception that makes it look like a server-side problem!
544        deleteTmpFiles(localPaths);
545        throw new IOException("File " + url + " received length " + received +
546                              " is not of the advertised size " +
547                              advertisedSize);
548      }
549    }
550    double xferSec = Math.max(
551        ((float)(Time.monotonicNow() - startTime)) / 1000.0, 0.001);
552    long xferKb = received / 1024;
553    LOG.info(String.format("Transfer took %.2fs at %.2f KB/s",
554        xferSec, xferKb / xferSec));
555
556    if (digester != null) {
557      MD5Hash computedDigest = new MD5Hash(digester.digest());
558      
559      if (advertisedDigest != null &&
560          !computedDigest.equals(advertisedDigest)) {
561        deleteTmpFiles(localPaths);
562        throw new IOException("File " + url + " computed digest " +
563            computedDigest + " does not match advertised digest " + 
564            advertisedDigest);
565      }
566      return computedDigest;
567    } else {
568      return null;
569    }    
570  }
571
572  private static void deleteTmpFiles(List<File> files) {
573    if (files == null) {
574      return;
575    }
576
577    LOG.info("Deleting temporary files: " + files);
578    for (File file : files) {
579      if (!file.delete()) {
580        LOG.warn("Deleting " + file + " has failed");
581      }
582    }
583  }
584
585  private static MD5Hash parseMD5Header(HttpURLConnection connection) {
586    String header = connection.getHeaderField(MD5_HEADER);
587    return (header != null) ? new MD5Hash(header) : null;
588  }
589
590  private static MD5Hash parseMD5Header(HttpServletRequest request) {
591    String header = request.getHeader(MD5_HEADER);
592    return (header != null) ? new MD5Hash(header) : null;
593  }
594
595  public static class HttpGetFailedException extends IOException {
596    private static final long serialVersionUID = 1L;
597    private final int responseCode;
598
599    HttpGetFailedException(String msg, HttpURLConnection connection) throws IOException {
600      super(msg);
601      this.responseCode = connection.getResponseCode();
602    }
603    
604    public int getResponseCode() {
605      return responseCode;
606    }
607  }
608
609  public static class HttpPutFailedException extends IOException {
610    private static final long serialVersionUID = 1L;
611    private final int responseCode;
612
613    HttpPutFailedException(String msg, int responseCode) throws IOException {
614      super(msg);
615      this.responseCode = responseCode;
616    }
617
618    public int getResponseCode() {
619      return responseCode;
620    }
621  }
622
623}