You are viewing a plain text version of this content. The canonical link for it is here.
Posted to common-commits@hadoop.apache.org by cu...@apache.org on 2007/08/29 20:02:42 UTC

svn commit: r570881 - in /lucene/hadoop/trunk: ./ src/java/org/apache/hadoop/io/ src/test/org/apache/hadoop/io/

Author: cutting
Date: Wed Aug 29 11:02:41 2007
New Revision: 570881

URL: http://svn.apache.org/viewvc?rev=570881&view=rev
Log:
HADOOP-1775.  Fix a NullPointerException and an IllegalArgumentException in MapWritable.  Contributed by Jim Kellerman.

Modified:
    lucene/hadoop/trunk/CHANGES.txt
    lucene/hadoop/trunk/src/java/org/apache/hadoop/io/AbstractMapWritable.java
    lucene/hadoop/trunk/src/java/org/apache/hadoop/io/MapWritable.java
    lucene/hadoop/trunk/src/java/org/apache/hadoop/io/SortedMapWritable.java
    lucene/hadoop/trunk/src/test/org/apache/hadoop/io/TestMapWritable.java
    lucene/hadoop/trunk/src/test/org/apache/hadoop/io/TestSortedMapWritable.java

Modified: lucene/hadoop/trunk/CHANGES.txt
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/CHANGES.txt?rev=570881&r1=570880&r2=570881&view=diff
==============================================================================
--- lucene/hadoop/trunk/CHANGES.txt (original)
+++ lucene/hadoop/trunk/CHANGES.txt Wed Aug 29 11:02:41 2007
@@ -69,6 +69,10 @@
     HADOOP-1748.  Fix tasktracker to be able to launch tasks when log
     directory is relative.  (omalley via cutting)
 
+    HADOOP-1775 Fix a NullPointerException and an
+    IllegalArgumentException in MapWritable.
+    (Jim Kellerman via cutting)
+
   IMPROVEMENTS
 
     HADOOP-1756. Add toString() to some Writable-s. (ab)

Modified: lucene/hadoop/trunk/src/java/org/apache/hadoop/io/AbstractMapWritable.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/java/org/apache/hadoop/io/AbstractMapWritable.java?rev=570881&r1=570880&r2=570881&view=diff
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/io/AbstractMapWritable.java (original)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/io/AbstractMapWritable.java Wed Aug 29 11:02:41 2007
@@ -19,10 +19,11 @@
  */
 package org.apache.hadoop.io;
 
+import java.io.DataInput;
+import java.io.DataOutput;
 import java.io.IOException;
 import java.util.Map;
 import java.util.concurrent.ConcurrentHashMap;
-import java.util.concurrent.atomic.AtomicInteger;
 import java.util.concurrent.atomic.AtomicReference;
 
 import org.apache.hadoop.conf.Configurable;
@@ -49,20 +50,25 @@
   @SuppressWarnings("unchecked")
   private Map<Byte, Class> idToClassMap = new ConcurrentHashMap<Byte, Class>();
   
-  /* The number of known classes (established by the constructor) */
-  private AtomicInteger newClasses = new AtomicInteger(0);
+  /* The number of new classes (those not established by the constructor) */
+  private volatile byte newClasses = 0;
   
   /** @return the number of known classes */
-  protected int getNewClasses() {
-    return newClasses.get();
+  byte getNewClasses() {
+    return newClasses;
   }
 
-  /** used to add "predefined" classes */
+  /**
+   * Used to add "predefined" classes and by Writable to copy "new" classes.
+   */
   @SuppressWarnings("unchecked")
-  protected void addToMap(Class clazz, byte id) {
+  private synchronized void addToMap(Class clazz, byte id) {
     if (classToIdMap.containsKey(clazz)) {
-      throw new IllegalArgumentException ("Class " + clazz.getName() +
-          " already registered");
+      byte b = classToIdMap.get(clazz);
+      if (b != id) {
+        throw new IllegalArgumentException ("Class " + clazz.getName() +
+          " already registered but maps to " + b + " and not " + id);
+      }
     }
     if (idToClassMap.containsKey(id)) {
       Class c = idToClassMap.get(id);
@@ -77,15 +83,15 @@
   
   /** Add a Class to the maps if it is not already present. */ 
   @SuppressWarnings("unchecked")
-  protected void addToMap(Class clazz) {
+  protected synchronized void addToMap(Class clazz) {
     if (classToIdMap.containsKey(clazz)) {
       return;
     }
-    byte id = Integer.valueOf((newClasses.incrementAndGet())).byteValue();
-    if (id > Byte.MAX_VALUE) {
+    if (newClasses + 1 > Byte.MAX_VALUE) {
       throw new IndexOutOfBoundsException("adding an additional class would" +
       " exceed the maximum number allowed");
     }
+    byte id = ++newClasses;
     addToMap(clazz, id);
   }
 
@@ -102,7 +108,7 @@
   }
 
   /** Used by child copy constructors. */
-  protected void copy(Writable other) {
+  protected synchronized void copy(Writable other) {
     if (other != null) {
       try {
         DataOutputBuffer out = new DataOutputBuffer();
@@ -161,17 +167,49 @@
 
   }
 
-  /**
-   * @return the conf
-   */
+  /** @return the conf */
   public Configuration getConf() {
     return conf.get();
   }
 
-  /**
-   * @param conf the conf to set
-   */
+  /** @param conf the conf to set */
   public void setConf(Configuration conf) {
     this.conf.set(conf);
   }
+  
+  /** {@inheritDoc} */
+  public void write(DataOutput out) throws IOException {
+    
+    // First write out the size of the class table and any classes that are
+    // "unknown" classes
+    
+    out.writeByte(newClasses);
+
+    for (byte i = 1; i <= newClasses; i++) {
+      out.writeByte(i);
+      out.writeUTF(getClass(i).getName());
+    }
+  }
+  
+  /** {@inheritDoc} */
+  public void readFields(DataInput in) throws IOException {
+    
+    // Get the number of "unknown" classes
+    
+    newClasses = in.readByte();
+    
+    // Then read in the class names and add them to our tables
+    
+    for (int i = 0; i < newClasses; i++) {
+      byte id = in.readByte();
+      String className = in.readUTF();
+      try {
+        addToMap(Class.forName(className), id);
+        
+      } catch (ClassNotFoundException e) {
+        throw new IOException("can't find class: " + className + " because "+
+            e.getMessage());
+      }
+    }
+  }    
 }

Modified: lucene/hadoop/trunk/src/java/org/apache/hadoop/io/MapWritable.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/java/org/apache/hadoop/io/MapWritable.java?rev=570881&r1=570880&r2=570881&view=diff
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/io/MapWritable.java (original)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/io/MapWritable.java Wed Aug 29 11:02:41 2007
@@ -121,22 +121,14 @@
   // Writable
   
   /** {@inheritDoc} */
+  @Override
   public void write(DataOutput out) throws IOException {
-    
-    // First write out the size of the class table and any classes that are
-    // not "known" classes
-    
-    byte newClasses = Integer.valueOf(getNewClasses()).byteValue();
-    out.writeByte(newClasses);
-    for (byte i = 1; i <= newClasses; i++) {
-      out.writeByte(i);
-      out.writeUTF(getClass(i).getName());
-    }
+    super.write(out);
     
     // Write out the number of entries in the map
     
     out.writeInt(instance.size());
-    
+
     // Then write out each key/value pair
     
     for (Map.Entry<Writable, Writable> e: instance.entrySet()) {
@@ -148,26 +140,9 @@
   }
 
   /** {@inheritDoc} */
-  @SuppressWarnings("unchecked")
+  @Override
   public void readFields(DataInput in) throws IOException {
-    
-    // Get the number of "unknown" classes
-    
-    byte newClasses = in.readByte();
-    
-    // Then read in the class names and add them to our parent's class tables
-    
-    for (int i = 0; i < newClasses; i++) {
-      byte id = in.readByte();
-      String className = in.readUTF();
-      try {
-        addToMap(Class.forName(className), id);
-        
-      } catch (ClassNotFoundException e) {
-        throw new IOException("can't find class: " + className + " because "+
-            e.getMessage());
-      }
-    }
+    super.readFields(in);
     
     // Read the number of entries in the map
     

Modified: lucene/hadoop/trunk/src/java/org/apache/hadoop/io/SortedMapWritable.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/java/org/apache/hadoop/io/SortedMapWritable.java?rev=570881&r1=570880&r2=570881&view=diff
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/io/SortedMapWritable.java (original)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/io/SortedMapWritable.java Wed Aug 29 11:02:41 2007
@@ -159,25 +159,9 @@
   }
 
   /** {@inheritDoc} */
+  @Override
   public void readFields(DataInput in) throws IOException {
-    
-    // Get the number of "unknown" classes
-    
-    byte newClasses = in.readByte();
-    
-    // Then read in the class names and add them to our parent's class tables
-    
-    for (int i = 0; i < newClasses; i++) {
-      byte id = in.readByte();
-      String className = in.readUTF();
-      try {
-        addToMap(Class.forName(className), id);
-        
-      } catch (ClassNotFoundException e) {
-        throw new IOException("can't find class: " + className + " because "+
-            e.getMessage());
-      }
-    }
+    super.readFields(in);
     
     // Read the number of entries in the map
     
@@ -201,17 +185,9 @@
   }
 
   /** {@inheritDoc} */
+  @Override
   public void write(DataOutput out) throws IOException {
-    
-    // First write out the size of the class table and any classes that are
-    // not "known" classes
-
-    byte newClasses = Integer.valueOf(getNewClasses()).byteValue();
-    out.writeByte(newClasses);
-    for (byte i = 1; i <= newClasses; i++) {
-      out.writeByte(i);
-      out.writeUTF(getClass(i).getName());
-    }
+    super.write(out);
     
     // Write out the number of entries in the map
     

Modified: lucene/hadoop/trunk/src/test/org/apache/hadoop/io/TestMapWritable.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/test/org/apache/hadoop/io/TestMapWritable.java?rev=570881&r1=570880&r2=570881&view=diff
==============================================================================
--- lucene/hadoop/trunk/src/test/org/apache/hadoop/io/TestMapWritable.java (original)
+++ lucene/hadoop/trunk/src/test/org/apache/hadoop/io/TestMapWritable.java Wed Aug 29 11:02:41 2007
@@ -84,4 +84,17 @@
       }
     }
   }
+
+  /**
+   * Test that number of "unknown" classes is propagated across multiple copies.
+   */
+  @SuppressWarnings("deprecation")
+  public void testForeignClass() {
+    MapWritable inMap = new MapWritable();
+    inMap.put(new Text("key"), new UTF8("value"));
+    inMap.put(new Text("key2"), new UTF8("value2"));
+    MapWritable outMap = new MapWritable(inMap);
+    MapWritable copyOfCopy = new MapWritable(outMap);
+    assertEquals(1, copyOfCopy.getNewClasses());
+  }
 }

Modified: lucene/hadoop/trunk/src/test/org/apache/hadoop/io/TestSortedMapWritable.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/test/org/apache/hadoop/io/TestSortedMapWritable.java?rev=570881&r1=570880&r2=570881&view=diff
==============================================================================
--- lucene/hadoop/trunk/src/test/org/apache/hadoop/io/TestSortedMapWritable.java (original)
+++ lucene/hadoop/trunk/src/test/org/apache/hadoop/io/TestSortedMapWritable.java Wed Aug 29 11:02:41 2007
@@ -88,4 +88,17 @@
       }
     }
   }
+  
+  /**
+   * Test that number of "unknown" classes is propagated across multiple copies.
+   */
+  @SuppressWarnings("deprecation")
+  public void testForeignClass() {
+    SortedMapWritable inMap = new SortedMapWritable();
+    inMap.put(new Text("key"), new UTF8("value"));
+    inMap.put(new Text("key2"), new UTF8("value2"));
+    SortedMapWritable outMap = new SortedMapWritable(inMap);
+    SortedMapWritable copyOfCopy = new SortedMapWritable(outMap);
+    assertEquals(1, copyOfCopy.getNewClasses());
+  }
 }