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