001    /**
002     * Licensed to the Apache Software Foundation (ASF) under one
003     * or more contributor license agreements.  See the NOTICE file
004     * distributed with this work for additional information
005     * regarding copyright ownership.  The ASF licenses this file
006     * to you under the Apache License, Version 2.0 (the
007     * "License"); you may not use this file except in compliance
008     * with the License.  You may obtain a copy of the License at
009     *
010     *     http://www.apache.org/licenses/LICENSE-2.0
011     *
012     * Unless required by applicable law or agreed to in writing, software
013     * distributed under the License is distributed on an "AS IS" BASIS,
014     * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
015     * See the License for the specific language governing permissions and
016     * limitations under the License.
017     */
018    package org.apache.hadoop.hdfs.server.namenode;
019    
020    import java.io.*;
021    import java.net.*;
022    import java.security.DigestInputStream;
023    import java.security.MessageDigest;
024    import java.util.ArrayList;
025    import java.util.List;
026    import java.lang.Math;
027    
028    import javax.servlet.ServletOutputStream;
029    import javax.servlet.ServletResponse;
030    import javax.servlet.http.HttpServletResponse;
031    
032    import org.apache.commons.logging.Log;
033    import org.apache.commons.logging.LogFactory;
034    import org.apache.hadoop.classification.InterfaceAudience;
035    import org.apache.hadoop.conf.Configuration;
036    import org.apache.hadoop.fs.FileUtil;
037    import org.apache.hadoop.http.HttpConfig;
038    import org.apache.hadoop.security.SecurityUtil;
039    import org.apache.hadoop.util.Time;
040    import org.apache.hadoop.hdfs.DFSConfigKeys;
041    import org.apache.hadoop.hdfs.HdfsConfiguration;
042    import org.apache.hadoop.hdfs.protocol.HdfsConstants;
043    import org.apache.hadoop.hdfs.server.common.StorageErrorReporter;
044    import org.apache.hadoop.hdfs.server.common.Storage;
045    import org.apache.hadoop.hdfs.server.common.Storage.StorageDirectory;
046    import org.apache.hadoop.hdfs.server.namenode.NNStorage.NameNodeDirType;
047    import org.apache.hadoop.hdfs.server.protocol.RemoteEditLog;
048    import org.apache.hadoop.hdfs.util.DataTransferThrottler;
049    import org.apache.hadoop.io.MD5Hash;
050    
051    import com.google.common.annotations.VisibleForTesting;
052    import com.google.common.collect.Lists;
053    
054    
055    /**
056     * This class provides fetching a specified file from the NameNode.
057     */
058    @InterfaceAudience.Private
059    public class TransferFsImage {
060      
061      public final static String CONTENT_LENGTH = "Content-Length";
062      public final static String MD5_HEADER = "X-MD5-Digest";
063      @VisibleForTesting
064      static int timeout = 0;
065    
066      private static final Log LOG = LogFactory.getLog(TransferFsImage.class);
067      
068      public static void downloadMostRecentImageToDirectory(String fsName,
069          File dir) throws IOException {
070        String fileId = GetImageServlet.getParamStringForMostRecentImage();
071        getFileClient(fsName, fileId, Lists.newArrayList(dir),
072            null, false);
073      }
074    
075      public static MD5Hash downloadImageToStorage(
076          String fsName, long imageTxId, Storage dstStorage, boolean needDigest)
077          throws IOException {
078        String fileid = GetImageServlet.getParamStringForImage(
079            imageTxId, dstStorage);
080        String fileName = NNStorage.getCheckpointImageFileName(imageTxId);
081        
082        List<File> dstFiles = dstStorage.getFiles(
083            NameNodeDirType.IMAGE, fileName);
084        if (dstFiles.isEmpty()) {
085          throw new IOException("No targets in destination storage!");
086        }
087        
088        MD5Hash hash = getFileClient(fsName, fileid, dstFiles, dstStorage, needDigest);
089        LOG.info("Downloaded file " + dstFiles.get(0).getName() + " size " +
090            dstFiles.get(0).length() + " bytes.");
091        return hash;
092      }
093      
094      static void downloadEditsToStorage(String fsName, RemoteEditLog log,
095          NNStorage dstStorage) throws IOException {
096        assert log.getStartTxId() > 0 && log.getEndTxId() > 0 :
097          "bad log: " + log;
098        String fileid = GetImageServlet.getParamStringForLog(
099            log, dstStorage);
100        String finalFileName = NNStorage.getFinalizedEditsFileName(
101            log.getStartTxId(), log.getEndTxId());
102    
103        List<File> finalFiles = dstStorage.getFiles(NameNodeDirType.EDITS,
104            finalFileName);
105        assert !finalFiles.isEmpty() : "No checkpoint targets.";
106        
107        for (File f : finalFiles) {
108          if (f.exists() && FileUtil.canRead(f)) {
109            LOG.info("Skipping download of remote edit log " +
110                log + " since it already is stored locally at " + f);
111            return;
112          } else if (LOG.isDebugEnabled()) {
113            LOG.debug("Dest file: " + f);
114          }
115        }
116    
117        final long milliTime = System.currentTimeMillis();
118        String tmpFileName = NNStorage.getTemporaryEditsFileName(
119            log.getStartTxId(), log.getEndTxId(), milliTime);
120        List<File> tmpFiles = dstStorage.getFiles(NameNodeDirType.EDITS,
121            tmpFileName);
122        getFileClient(fsName, fileid, tmpFiles, dstStorage, false);
123        LOG.info("Downloaded file " + tmpFiles.get(0).getName() + " size " +
124            finalFiles.get(0).length() + " bytes.");
125    
126        CheckpointFaultInjector.getInstance().beforeEditsRename();
127    
128        for (StorageDirectory sd : dstStorage.dirIterable(NameNodeDirType.EDITS)) {
129          File tmpFile = NNStorage.getTemporaryEditsFile(sd,
130              log.getStartTxId(), log.getEndTxId(), milliTime);
131          File finalizedFile = NNStorage.getFinalizedEditsFile(sd,
132              log.getStartTxId(), log.getEndTxId());
133          if (LOG.isDebugEnabled()) {
134            LOG.debug("Renaming " + tmpFile + " to " + finalizedFile);
135          }
136          boolean success = tmpFile.renameTo(finalizedFile);
137          if (!success) {
138            LOG.warn("Unable to rename edits file from " + tmpFile
139                + " to " + finalizedFile);
140          }
141        }
142      }
143     
144      /**
145       * Requests that the NameNode download an image from this node.
146       *
147       * @param fsName the http address for the remote NN
148       * @param imageListenAddress the host/port where the local node is running an
149       *                           HTTPServer hosting GetImageServlet
150       * @param storage the storage directory to transfer the image from
151       * @param txid the transaction ID of the image to be uploaded
152       */
153      public static void uploadImageFromStorage(String fsName,
154          InetSocketAddress imageListenAddress,
155          Storage storage, long txid) throws IOException {
156        
157        String fileid = GetImageServlet.getParamStringToPutImage(
158            txid, imageListenAddress, storage);
159        // this doesn't directly upload an image, but rather asks the NN
160        // to connect back to the 2NN to download the specified image.
161        try {
162          TransferFsImage.getFileClient(fsName, fileid, null, null, false);
163        } catch (HttpGetFailedException e) {
164          if (e.getResponseCode() == HttpServletResponse.SC_CONFLICT) {
165            // this is OK - this means that a previous attempt to upload
166            // this checkpoint succeeded even though we thought it failed.
167            LOG.info("Image upload with txid " + txid + 
168                " conflicted with a previous image upload to the " +
169                "same NameNode. Continuing...", e);
170            return;
171          } else {
172            throw e;
173          }
174        }
175        LOG.info("Uploaded image with txid " + txid + " to namenode at " +
176                    fsName);
177      }
178    
179      
180      /**
181       * A server-side method to respond to a getfile http request
182       * Copies the contents of the local file into the output stream.
183       */
184      public static void getFileServer(ServletResponse response, File localfile,
185          FileInputStream infile,
186          DataTransferThrottler throttler) 
187        throws IOException {
188        byte buf[] = new byte[HdfsConstants.IO_FILE_BUFFER_SIZE];
189        ServletOutputStream out = null;
190        try {
191          CheckpointFaultInjector.getInstance()
192              .aboutToSendFile(localfile);
193          out = response.getOutputStream();
194    
195          if (CheckpointFaultInjector.getInstance().
196                shouldSendShortFile(localfile)) {
197              // Test sending image shorter than localfile
198              long len = localfile.length();
199              buf = new byte[(int)Math.min(len/2, HdfsConstants.IO_FILE_BUFFER_SIZE)];
200              // This will read at most half of the image
201              // and the rest of the image will be sent over the wire
202              infile.read(buf);
203          }
204          int num = 1;
205          while (num > 0) {
206            num = infile.read(buf);
207            if (num <= 0) {
208              break;
209            }
210            if (CheckpointFaultInjector.getInstance()
211                  .shouldCorruptAByte(localfile)) {
212              // Simulate a corrupted byte on the wire
213              LOG.warn("SIMULATING A CORRUPT BYTE IN IMAGE TRANSFER!");
214              buf[0]++;
215            }
216            
217            out.write(buf, 0, num);
218            if (throttler != null) {
219              throttler.throttle(num);
220            }
221          }
222        } finally {
223          if (out != null) {
224            out.close();
225          }
226        }
227      }
228    
229      /**
230       * Client-side Method to fetch file from a server
231       * Copies the response from the URL to a list of local files.
232       * @param dstStorage if an error occurs writing to one of the files,
233       *                   this storage object will be notified. 
234       * @Return a digest of the received file if getChecksum is true
235       */
236      static MD5Hash getFileClient(String nnHostPort,
237          String queryString, List<File> localPaths,
238          Storage dstStorage, boolean getChecksum) throws IOException {
239    
240        String str = HttpConfig.getSchemePrefix() + nnHostPort + "/getimage?" +
241            queryString;
242        LOG.info("Opening connection to " + str);
243        //
244        // open connection to remote server
245        //
246        URL url = new URL(str);
247        return doGetUrl(url, localPaths, dstStorage, getChecksum);
248      }
249      
250      public static MD5Hash doGetUrl(URL url, List<File> localPaths,
251          Storage dstStorage, boolean getChecksum) throws IOException {
252        long startTime = Time.monotonicNow();
253    
254        HttpURLConnection connection = (HttpURLConnection)
255          SecurityUtil.openSecureHttpConnection(url);
256    
257        if (timeout <= 0) {
258          Configuration conf = new HdfsConfiguration();
259          timeout = conf.getInt(DFSConfigKeys.DFS_IMAGE_TRANSFER_TIMEOUT_KEY,
260              DFSConfigKeys.DFS_IMAGE_TRANSFER_TIMEOUT_DEFAULT);
261        }
262    
263        if (timeout > 0) {
264          connection.setConnectTimeout(timeout);
265          connection.setReadTimeout(timeout);
266        }
267    
268        if (connection.getResponseCode() != HttpURLConnection.HTTP_OK) {
269          throw new HttpGetFailedException(
270              "Image transfer servlet at " + url +
271              " failed with status code " + connection.getResponseCode() +
272              "\nResponse message:\n" + connection.getResponseMessage(),
273              connection);
274        }
275        
276        long advertisedSize;
277        String contentLength = connection.getHeaderField(CONTENT_LENGTH);
278        if (contentLength != null) {
279          advertisedSize = Long.parseLong(contentLength);
280        } else {
281          throw new IOException(CONTENT_LENGTH + " header is not provided " +
282                                "by the namenode when trying to fetch " + url);
283        }
284        
285        if (localPaths != null) {
286          String fsImageName = connection.getHeaderField(
287              GetImageServlet.HADOOP_IMAGE_EDITS_HEADER);
288          // If the local paths refer to directories, use the server-provided header
289          // as the filename within that directory
290          List<File> newLocalPaths = new ArrayList<File>();
291          for (File localPath : localPaths) {
292            if (localPath.isDirectory()) {
293              if (fsImageName == null) {
294                throw new IOException("No filename header provided by server");
295              }
296              newLocalPaths.add(new File(localPath, fsImageName));
297            } else {
298              newLocalPaths.add(localPath);
299            }
300          }
301          localPaths = newLocalPaths;
302        }
303        
304        MD5Hash advertisedDigest = parseMD5Header(connection);
305    
306        long received = 0;
307        InputStream stream = connection.getInputStream();
308        MessageDigest digester = null;
309        if (getChecksum) {
310          digester = MD5Hash.getDigester();
311          stream = new DigestInputStream(stream, digester);
312        }
313        boolean finishedReceiving = false;
314    
315        List<FileOutputStream> outputStreams = Lists.newArrayList();
316    
317        try {
318          if (localPaths != null) {
319            for (File f : localPaths) {
320              try {
321                if (f.exists()) {
322                  LOG.warn("Overwriting existing file " + f
323                      + " with file downloaded from " + url);
324                }
325                outputStreams.add(new FileOutputStream(f));
326              } catch (IOException ioe) {
327                LOG.warn("Unable to download file " + f, ioe);
328                // This will be null if we're downloading the fsimage to a file
329                // outside of an NNStorage directory.
330                if (dstStorage != null &&
331                    (dstStorage instanceof StorageErrorReporter)) {
332                  ((StorageErrorReporter)dstStorage).reportErrorOnFile(f);
333                }
334              }
335            }
336            
337            if (outputStreams.isEmpty()) {
338              throw new IOException(
339                  "Unable to download to any storage directory");
340            }
341          }
342          
343          int num = 1;
344          byte[] buf = new byte[HdfsConstants.IO_FILE_BUFFER_SIZE];
345          while (num > 0) {
346            num = stream.read(buf);
347            if (num > 0) {
348              received += num;
349              for (FileOutputStream fos : outputStreams) {
350                fos.write(buf, 0, num);
351              }
352            }
353          }
354          finishedReceiving = true;
355        } finally {
356          stream.close();
357          for (FileOutputStream fos : outputStreams) {
358            fos.getChannel().force(true);
359            fos.close();
360          }
361          if (finishedReceiving && received != advertisedSize) {
362            // only throw this exception if we think we read all of it on our end
363            // -- otherwise a client-side IOException would be masked by this
364            // exception that makes it look like a server-side problem!
365            throw new IOException("File " + url + " received length " + received +
366                                  " is not of the advertised size " +
367                                  advertisedSize);
368          }
369        }
370        double xferSec = Math.max(
371            ((float)(Time.monotonicNow() - startTime)) / 1000.0, 0.001);
372        long xferKb = received / 1024;
373        LOG.info(String.format("Transfer took %.2fs at %.2f KB/s",
374            xferSec, xferKb / xferSec));
375    
376        if (digester != null) {
377          MD5Hash computedDigest = new MD5Hash(digester.digest());
378          
379          if (advertisedDigest != null &&
380              !computedDigest.equals(advertisedDigest)) {
381            throw new IOException("File " + url + " computed digest " +
382                computedDigest + " does not match advertised digest " + 
383                advertisedDigest);
384          }
385          return computedDigest;
386        } else {
387          return null;
388        }    
389      }
390    
391      private static MD5Hash parseMD5Header(HttpURLConnection connection) {
392        String header = connection.getHeaderField(MD5_HEADER);
393        return (header != null) ? new MD5Hash(header) : null;
394      }
395      
396      public static class HttpGetFailedException extends IOException {
397        private static final long serialVersionUID = 1L;
398        private final int responseCode;
399    
400        HttpGetFailedException(String msg, HttpURLConnection connection) throws IOException {
401          super(msg);
402          this.responseCode = connection.getResponseCode();
403        }
404        
405        public int getResponseCode() {
406          return responseCode;
407        }
408      }
409    
410    }