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