hadoop-common-commits mailing list archives

Site index · List index
Message view « Date » · « Thread »
Top « Date » · « Thread »
From cutt...@apache.org
Subject svn commit: r546240 - in /lucene/hadoop/trunk: ./ src/contrib/streaming/src/java/org/apache/hadoop/streaming/ src/java/org/apache/hadoop/dfs/ src/java/org/apache/hadoop/fs/ src/java/org/apache/hadoop/io/ src/java/org/apache/hadoop/mapred/ src/java/org/...
Date Mon, 11 Jun 2007 20:11:33 GMT
Author: cutting
Date: Mon Jun 11 13:11:31 2007
New Revision: 546240

URL: http://svn.apache.org/viewvc?view=rev&rev=546240
Log:
HADOOP-1414.  Fix a number of issues identified by FindBugs as 'Bad Practice.'  Contributed
by Dhruba.

Modified:
    lucene/hadoop/trunk/CHANGES.txt
    lucene/hadoop/trunk/src/contrib/streaming/src/java/org/apache/hadoop/streaming/StreamJob.java
    lucene/hadoop/trunk/src/contrib/streaming/src/java/org/apache/hadoop/streaming/StreamUtil.java
    lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/Block.java
    lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/DatanodeID.java
    lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSEditLog.java
    lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSImage.java
    lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSNamesystem.java
    lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/Storage.java
    lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/TransferFsImage.java
    lucene/hadoop/trunk/src/java/org/apache/hadoop/fs/FsShell.java
    lucene/hadoop/trunk/src/java/org/apache/hadoop/fs/LocalFileSystem.java
    lucene/hadoop/trunk/src/java/org/apache/hadoop/io/SequenceFile.java
    lucene/hadoop/trunk/src/java/org/apache/hadoop/io/Text.java
    lucene/hadoop/trunk/src/java/org/apache/hadoop/mapred/JobClient.java
    lucene/hadoop/trunk/src/java/org/apache/hadoop/mapred/JobEndNotifier.java
    lucene/hadoop/trunk/src/java/org/apache/hadoop/mapred/JobHistory.java
    lucene/hadoop/trunk/src/java/org/apache/hadoop/mapred/TaskLog.java
    lucene/hadoop/trunk/src/java/org/apache/hadoop/record/Buffer.java
    lucene/hadoop/trunk/src/java/org/apache/hadoop/util/CopyFiles.java
    lucene/hadoop/trunk/src/java/org/apache/hadoop/util/HostsFileReader.java

Modified: lucene/hadoop/trunk/CHANGES.txt
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/CHANGES.txt?view=diff&rev=546240&r1=546239&r2=546240
==============================================================================
--- lucene/hadoop/trunk/CHANGES.txt (original)
+++ lucene/hadoop/trunk/CHANGES.txt Mon Jun 11 13:11:31 2007
@@ -81,6 +81,10 @@
  26. HADOOP-1446.  Update the TaskTracker metrics while the task is
      running. (Devaraj via omalley)
 
+ 27. HADOOP-1414.  Fix a number of issues identified by FindBugs as
+     "Bad Practice".  (Dhruba Borthakur via cutting)
+
+
 Release 0.13.0 - 2007-06-08
 
  1. HADOOP-1047.  Fix TestReplication to succeed more reliably.

Modified: lucene/hadoop/trunk/src/contrib/streaming/src/java/org/apache/hadoop/streaming/StreamJob.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/contrib/streaming/src/java/org/apache/hadoop/streaming/StreamJob.java?view=diff&rev=546240&r1=546239&r2=546240
==============================================================================
--- lucene/hadoop/trunk/src/contrib/streaming/src/java/org/apache/hadoop/streaming/StreamJob.java
(original)
+++ lucene/hadoop/trunk/src/contrib/streaming/src/java/org/apache/hadoop/streaming/StreamJob.java
Mon Jun 11 13:11:31 2007
@@ -526,7 +526,7 @@
   public void fail(String message) {
     if (mayExit_) {
       System.err.println(message);
-      System.exit(1);
+      throw new RuntimeException(message);
     } else {
       throw new IllegalArgumentException(message);
     }

Modified: lucene/hadoop/trunk/src/contrib/streaming/src/java/org/apache/hadoop/streaming/StreamUtil.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/contrib/streaming/src/java/org/apache/hadoop/streaming/StreamUtil.java?view=diff&rev=546240&r1=546239&r2=546240
==============================================================================
--- lucene/hadoop/trunk/src/contrib/streaming/src/java/org/apache/hadoop/streaming/StreamUtil.java
(original)
+++ lucene/hadoop/trunk/src/contrib/streaming/src/java/org/apache/hadoop/streaming/StreamUtil.java
Mon Jun 11 13:11:31 2007
@@ -30,6 +30,7 @@
 
 import org.apache.hadoop.fs.Path;
 import org.apache.hadoop.fs.FileSystem;
+import org.apache.hadoop.fs.FSDataInputStream;
 import org.apache.hadoop.mapred.FileSplit;
 import org.apache.hadoop.mapred.JobConf;
 
@@ -320,10 +321,10 @@
   static String slurpHadoop(Path p, FileSystem fs) throws IOException {
     int len = (int) fs.getLength(p);
     byte[] buf = new byte[len];
-    InputStream in = fs.open(p);
+    FSDataInputStream in = fs.open(p);
     String contents = null;
     try {
-      in.read(buf, 0, len);
+      in.readFully(in.getPos(), buf);
       contents = new String(buf, "UTF-8");
     } finally {
       in.close();

Modified: lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/Block.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/Block.java?view=diff&rev=546240&r1=546239&r2=546240
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/Block.java (original)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/Block.java Mon Jun 11 13:11:31 2007
@@ -131,6 +131,9 @@
     }
   }
   public boolean equals(Object o) {
+    if (!(o instanceof Block)) {
+      return false;
+    }
     return blkid == ((Block)o).blkid;
   }
     

Modified: lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/DatanodeID.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/DatanodeID.java?view=diff&rev=546240&r1=546239&r2=546240
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/DatanodeID.java (original)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/DatanodeID.java Mon Jun 11 13:11:31
2007
@@ -97,6 +97,12 @@
   }
 
   public boolean equals(Object to) {
+    if (this == to) {
+      return true;
+    }
+    if (!(to instanceof DatanodeID)) {
+      return false;
+    }
     return (name.equals(((DatanodeID)to).getName()) &&
             storageID.equals(((DatanodeID)to).getStorageID()));
   }

Modified: lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSEditLog.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSEditLog.java?view=diff&rev=546240&r1=546239&r2=546240
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSEditLog.java (original)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSEditLog.java Mon Jun 11 13:11:31
2007
@@ -367,7 +367,7 @@
           } catch (IOException e) {
             FSNamesystem.LOG.error("Unable to append to edit log. " +
                                    "Fatal Error.");
-            System.exit(-1);
+            throw new RuntimeException("Unable to append to edit log. ");
           }
         }
       }

Modified: lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSImage.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSImage.java?view=diff&rev=546240&r1=546239&r2=546240
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSImage.java (original)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSImage.java Mon Jun 11 13:11:31 2007
@@ -937,6 +937,14 @@
       return node.compareTo(o);
     }
 
+    public boolean equals(Object o) {
+      return node.equals(o);
+    }
+
+    public int hashCode() {
+      return node.hashCode();
+    }
+
     /////////////////////////////////////////////////
     // Writable
     /////////////////////////////////////////////////

Modified: lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSNamesystem.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSNamesystem.java?view=diff&rev=546240&r1=546239&r2=546240
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSNamesystem.java (original)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/FSNamesystem.java Mon Jun 11 13:11:31
2007
@@ -1234,6 +1234,22 @@
         return l1.holder.compareTo(l2.holder);
       }
     }
+
+    public boolean equals(Object o) {
+      if (!(o instanceof Lease)) {
+        return false;
+      }
+      Lease obj = (Lease) o;
+      if (lastUpdate == obj.lastUpdate &&
+          holder.equals(obj.holder)) {
+        return true;
+      }
+      return false;
+    }
+
+    public int hashCode() {
+      return holder.hashCode();
+    }
   }
   /******************************************************
    * LeaseMonitor checks for leases that have expired,

Modified: lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/Storage.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/Storage.java?view=diff&rev=546240&r1=546239&r2=546240
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/Storage.java (original)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/Storage.java Mon Jun 11 13:11:31 2007
@@ -140,13 +140,17 @@
     
     void read(File from) throws IOException {
       RandomAccessFile file = new RandomAccessFile(from, "rws");
+      FileInputStream in = null;
       try {
-        FileInputStream in = new FileInputStream(file.getFD());
+        in = new FileInputStream(file.getFD());
         file.seek(0);
         Properties props = new Properties();
         props.load(in);
         getFields(props, this);
       } finally {
+        if (in != null) {
+          in.close();
+        }
         file.close();
       }
     }
@@ -165,11 +169,15 @@
       Properties props = new Properties();
       setFields(props, this);
       RandomAccessFile file = new RandomAccessFile(to, "rws");
+      FileOutputStream out = null;
       try {
         file.seek(0);
-        FileOutputStream out = new FileOutputStream(file.getFD());
+        out = new FileOutputStream(file.getFD());
         props.store(out, null);
       } finally {
+        if (out != null) {
+          out.close();
+        }
         file.close();
       }
     }

Modified: lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/TransferFsImage.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/TransferFsImage.java?view=diff&rev=546240&r1=546239&r2=546240
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/TransferFsImage.java (original)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/TransferFsImage.java Mon Jun 11 13:11:31
2007
@@ -134,14 +134,14 @@
     URLConnection connection = url.openConnection();
     InputStream stream = connection.getInputStream();
     FileOutputStream[] output = null;
-    if (localPath != null) {
-      output = new FileOutputStream[localPath.length];
-      for (int i = 0; i < output.length; i++) {
-        output[i] = new FileOutputStream(localPath[i]);
-      }
-    }
 
     try {
+      if (localPath != null) {
+        output = new FileOutputStream[localPath.length];
+        for (int i = 0; i < output.length; i++) {
+          output[i] = new FileOutputStream(localPath[i]);
+        }
+      }
       int num = 1;
       while (num > 0) {
         num = stream.read(buf);
@@ -155,7 +155,9 @@
       stream.close();
       if (localPath != null) {
         for (int i = 0; i < output.length; i++) {
-          output[i].close();
+          if (output[i] != null) {
+            output[i].close();
+          }
         }
       }
     }

Modified: lucene/hadoop/trunk/src/java/org/apache/hadoop/fs/FsShell.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/java/org/apache/hadoop/fs/FsShell.java?view=diff&rev=546240&r1=546239&r2=546240
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/fs/FsShell.java (original)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/fs/FsShell.java Mon Jun 11 13:11:31 2007
@@ -123,7 +123,7 @@
   void copyToLocal(String[]argv, int pos) throws IOException {
     if (argv.length-pos<2 || (argv.length-pos==2 && argv[pos].equalsIgnoreCase("-crc")))
{
       System.err.println("Usage: -get [-crc] <src> <dst>");
-      System.exit(-1);
+      throw new RuntimeException("Usage: -get [-crc] <src> <dst>");
     }
     boolean copyCrc = false;
     if ("-crc".equalsIgnoreCase(argv[pos])) {
@@ -221,7 +221,7 @@
   private void setReplication(String[] cmd, int pos) throws IOException {
     if (cmd.length-pos<2 || (cmd.length-pos==2 && cmd[pos].equalsIgnoreCase("-R")))
{
       System.err.println("Usage: [-R] <repvalue> <path>");
-      System.exit(-1);
+      throw new RuntimeException("Usage: [-R] <repvalue> <path>");
     }
       
     boolean recursive = false;
@@ -238,7 +238,7 @@
       pos++;
     } catch (NumberFormatException e) {
       System.err.println("Cannot set replication to: " + cmd[pos]);
-      System.exit(-1);
+      throw new RuntimeException("Cannot set replication to: " + cmd[pos]);
     }
       
     setReplication(rep, cmd[pos], recursive);

Modified: lucene/hadoop/trunk/src/java/org/apache/hadoop/fs/LocalFileSystem.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/java/org/apache/hadoop/fs/LocalFileSystem.java?view=diff&rev=546240&r1=546239&r2=546240
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/fs/LocalFileSystem.java (original)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/fs/LocalFileSystem.java Mon Jun 11 13:11:31
2007
@@ -29,6 +29,7 @@
  *****************************************************************/
 public class LocalFileSystem extends ChecksumFileSystem {
   static final URI NAME = URI.create("file:///");
+  static private Random rand = new Random();
 
   public LocalFileSystem() {
     super(new RawLocalFileSystem());
@@ -87,7 +88,7 @@
           throw new IOException("Mkdirs failed to create " + badDir.toString());
         }
       }
-      String suffix = "." + new Random().nextInt();
+      String suffix = "." + rand.nextInt();
       File badFile = new File(badDir, f.getName()+suffix);
       LOG.warn("Moving bad file " + f + " to " + badFile);
       in.close();                               // close it first

Modified: lucene/hadoop/trunk/src/java/org/apache/hadoop/io/SequenceFile.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/java/org/apache/hadoop/io/SequenceFile.java?view=diff&rev=546240&r1=546239&r2=546240
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/io/SequenceFile.java (original)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/io/SequenceFile.java Mon Jun 11 13:11:31
2007
@@ -576,6 +576,11 @@
       }
       return true;
     }
+
+    public int hashCode() {
+      assert false : "hashCode not designed";
+      return 42; // any arbitrary constant will do 
+    }
     
     public String toString() {
       StringBuffer sb = new StringBuffer();
@@ -1298,11 +1303,16 @@
                                          CompressionInputStream filter) throws IOException
{
       // Read data into a temporary buffer
       DataOutputBuffer dataBuffer = new DataOutputBuffer();
-      int dataBufferLength = WritableUtils.readVInt(in);
-      dataBuffer.write(in, dataBufferLength);
+
+      try {
+        int dataBufferLength = WritableUtils.readVInt(in);
+        dataBuffer.write(in, dataBufferLength);
       
-      // Set up 'buffer' connected to the input-stream
-      buffer.reset(dataBuffer.getData(), 0, dataBuffer.getLength());
+        // Set up 'buffer' connected to the input-stream
+        buffer.reset(dataBuffer.getData(), 0, dataBuffer.getLength());
+      } finally {
+        dataBuffer.close();
+      }
 
       // Reset the codec
       filter.resetState();
@@ -2548,6 +2558,24 @@
         }
         return (this.segmentPathName.toString()).
           compareTo(that.segmentPathName.toString());
+      }
+
+      public boolean equals(Object o) {
+        if (!(o instanceof SegmentDescriptor)) {
+          return false;
+        }
+        SegmentDescriptor that = (SegmentDescriptor)o;
+        if (this.segmentLength == that.segmentLength &&
+            this.segmentOffset == that.segmentOffset &&
+            this.segmentPathName.toString().equals(
+              that.segmentPathName.toString())) {
+          return true;
+        }
+        return false;
+      }
+
+      public int hashCode() {
+        return 37 * 17 + (int) (segmentOffset^(segmentOffset>>>32));
       }
 
       /** Fills up the rawKey object with the key returned by the Reader

Modified: lucene/hadoop/trunk/src/java/org/apache/hadoop/io/Text.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/java/org/apache/hadoop/io/Text.java?view=diff&rev=546240&r1=546239&r2=546240
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/io/Text.java (original)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/io/Text.java Mon Jun 11 13:11:31 2007
@@ -215,7 +215,7 @@
     try {
       return decode(bytes, 0, length);
     } catch (CharacterCodingException e) { 
-      return null;
+      throw new RuntimeException("Should not have happened " + e.toString()); 
     }
   }
   

Modified: lucene/hadoop/trunk/src/java/org/apache/hadoop/mapred/JobClient.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/java/org/apache/hadoop/mapred/JobClient.java?view=diff&rev=546240&r1=546239&r2=546240
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/mapred/JobClient.java (original)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/mapred/JobClient.java Mon Jun 11 13:11:31
2007
@@ -717,10 +717,11 @@
     
   public int run(String[] argv) throws Exception {
     if (argv.length < 2) {
-      System.out.println("JobClient -submit <job> | -status <id> |" + 
-                         " -events <id> |" +
-                         " -kill <id> [-jt <jobtracker:port>|<config>]");
-      System.exit(-1);
+      String cmd = "JobClient -submit <job> | -status <id> |" + 
+                   " -events <id> |" +
+                   " -kill <id> [-jt <jobtracker:port>|<config>]";
+      System.out.println(cmd);
+      throw new RuntimeException("JobClient:" + cmd);
     }
 
     // initialize JobClient

Modified: lucene/hadoop/trunk/src/java/org/apache/hadoop/mapred/JobEndNotifier.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/java/org/apache/hadoop/mapred/JobEndNotifier.java?view=diff&rev=546240&r1=546239&r2=546240
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/mapred/JobEndNotifier.java (original)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/mapred/JobEndNotifier.java Mon Jun 11 13:11:31
2007
@@ -196,7 +196,20 @@
       return (int)(delayTime - ((JobEndStatusInfo)d).delayTime);
     }
 
+    public boolean equals(Object o) {
+      if (!(o instanceof JobEndStatusInfo)) {
+        return false;
+      }
+      if (delayTime == ((JobEndStatusInfo)o).delayTime) {
+        return true;
+      }
+      return false;
+    }
 
+    public int hashCode() {
+      return 37 * 17 + (int) (delayTime^(delayTime>>>32));
+    }
+      
     public String toString() {
       return "URL: " + uri + " remaining retries: " + retryAttempts +
         " interval: " + retryInterval;

Modified: lucene/hadoop/trunk/src/java/org/apache/hadoop/mapred/JobHistory.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/java/org/apache/hadoop/mapred/JobHistory.java?view=diff&rev=546240&r1=546239&r2=546240
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/mapred/JobHistory.java (original)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/mapred/JobHistory.java Mon Jun 11 13:11:31
2007
@@ -115,15 +115,19 @@
    */
   public static void parseHistory(File path, Listener l) throws IOException{
     BufferedReader reader = new BufferedReader(new FileReader(path));
-    String line = null; 
-    StringBuffer buf = new StringBuffer(); 
-    while ((line = reader.readLine())!= null){
-      buf.append(line); 
-      if (!line.trim().endsWith("\"")){
-        continue; 
+    try {
+      String line = null; 
+      StringBuffer buf = new StringBuffer(); 
+      while ((line = reader.readLine())!= null){
+        buf.append(line); 
+        if (!line.trim().endsWith("\"")){
+          continue; 
+        }
+        parseLine(buf.toString(), l);
+        buf = new StringBuffer(); 
       }
-      parseLine(buf.toString(), l);
-      buf = new StringBuffer(); 
+    } finally {
+      try { reader.close(); } catch (IOException ex) {}
     }
   }
   /**

Modified: lucene/hadoop/trunk/src/java/org/apache/hadoop/mapred/TaskLog.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/java/org/apache/hadoop/mapred/TaskLog.java?view=diff&rev=546240&r1=546239&r2=546240
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/mapred/TaskLog.java (original)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/mapred/TaskLog.java Mon Jun 11 13:11:31
2007
@@ -402,21 +402,24 @@
       // Copy log data into buffer
       byte[] b = new byte[totalLogSize];
       SequenceInputStream in = new SequenceInputStream(streams.elements());
-      int bytesRead = 0, totalBytesRead = 0;
-      int off = 0, len = totalLogSize;
-      LOG.debug("Attempting to read " + len + " bytes from logs");
-      while ((bytesRead = in.read(b, off, len)) > 0) {
-        LOG.debug("Got " + bytesRead + " bytes");
-        off += bytesRead;
-        len -= bytesRead;
+      try {
+        int bytesRead = 0, totalBytesRead = 0;
+        int off = 0, len = totalLogSize;
+        LOG.debug("Attempting to read " + len + " bytes from logs");
+        while ((bytesRead = in.read(b, off, len)) > 0) {
+          LOG.debug("Got " + bytesRead + " bytes");
+          off += bytesRead;
+          len -= bytesRead;
         
-        totalBytesRead += bytesRead;
-      }
+          totalBytesRead += bytesRead;
+        }
 
-      if (totalBytesRead != totalLogSize) {
-        LOG.debug("Didn't not read all requisite data in logs!");
+        if (totalBytesRead != totalLogSize) {
+          LOG.debug("Didn't not read all requisite data in logs!");
+        }
+      } finally {
+        try { in.close(); } catch (IOException ex) {}
       }
-      
       return b;
     }
     
@@ -527,23 +530,28 @@
       
       // Copy requisite data into user buffer
       SequenceInputStream in = new SequenceInputStream(streams.elements());
-      if (streams.size() == (stopIndex - startIndex +1)) {
-        // Skip to get to 'logOffset' if logs haven't been purged
-        long skipBytes = 
-          in.skip(logOffset - indexRecords[startIndex].splitOffset);
-        LOG.debug("Skipped " + skipBytes + " bytes from " + 
-                  startIndex + " stream");
-      }
-      int bytesRead = 0, totalBytesRead = 0;
-      len = Math.min((int)logLength, len);
-      LOG.debug("Attempting to read " + len + " bytes from logs");
-      while ((bytesRead = in.read(b, off, len)) > 0) {
-        off += bytesRead;
-        len -= bytesRead;
+      int totalBytesRead = 0;
+      try {
+        if (streams.size() == (stopIndex - startIndex +1)) {
+          // Skip to get to 'logOffset' if logs haven't been purged
+          long skipBytes = 
+            in.skip(logOffset - indexRecords[startIndex].splitOffset);
+          LOG.debug("Skipped " + skipBytes + " bytes from " + 
+                    startIndex + " stream");
+        }
+        int bytesRead = 0;
+        len = Math.min((int)logLength, len);
+        LOG.debug("Attempting to read " + len + " bytes from logs");
+        while ((bytesRead = in.read(b, off, len)) > 0) {
+          off += bytesRead;
+          len -= bytesRead;
         
-        totalBytesRead += bytesRead;
+          totalBytesRead += bytesRead;
+        }
+      } finally {
+        try { in.close(); } catch (IOException e) {}
       }
-      
+
       return totalBytesRead;
     }
 

Modified: lucene/hadoop/trunk/src/java/org/apache/hadoop/record/Buffer.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/java/org/apache/hadoop/record/Buffer.java?view=diff&rev=546240&r1=546239&r2=546240
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/record/Buffer.java (original)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/record/Buffer.java Mon Jun 11 13:11:31
2007
@@ -240,6 +240,8 @@
   
   // inherit javadoc
   public Object clone() throws CloneNotSupportedException {
-    return new Buffer(this.get().clone(), 0, this.getCount());
+    Buffer result = (Buffer) super.clone();
+    result.copy(this.get(), 0, this.getCount());
+    return result;
   }
 }

Modified: lucene/hadoop/trunk/src/java/org/apache/hadoop/util/CopyFiles.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/java/org/apache/hadoop/util/CopyFiles.java?view=diff&rev=546240&r1=546239&r2=546240
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/util/CopyFiles.java (original)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/util/CopyFiles.java Mon Jun 11 13:11:31
2007
@@ -652,6 +652,10 @@
       if (fis != null) {
         fis.close();
       }
+    } finally {
+      if (fis != null) {
+        fis.close();
+      }
     }
 
     return !uris.isEmpty()? uris.toArray(new String[0]): null;

Modified: lucene/hadoop/trunk/src/java/org/apache/hadoop/util/HostsFileReader.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/java/org/apache/hadoop/util/HostsFileReader.java?view=diff&rev=546240&r1=546239&r2=546240
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/util/HostsFileReader.java (original)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/util/HostsFileReader.java Mon Jun 11 13:11:31
2007
@@ -23,8 +23,9 @@
 
   private void readFileToSet(String filename, Set<String> set) throws IOException {
     FileInputStream fis = new FileInputStream(new File(filename));
+    BufferedReader reader = null;
     try {
-      BufferedReader reader = new BufferedReader(new InputStreamReader(fis));
+      reader = new BufferedReader(new InputStreamReader(fis));
       String line;
       while ((line = reader.readLine()) != null) {
         String[] nodes = line.split("[ \t\n\f\r]+");
@@ -35,6 +36,9 @@
         }
       }   
     } finally {
+      if (reader != null) {
+        reader.close();
+      }
       fis.close();
     }  
   }



Mime
View raw message