You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@hama.apache.org by ed...@apache.org on 2008/09/03 19:07:56 UTC

svn commit: r691689 - in /incubator/hama/trunk/src: java/org/apache/hama/ java/org/apache/hama/algebra/ java/org/apache/hama/io/ test/org/apache/hama/

Author: edwardyoon
Date: Wed Sep  3 10:07:55 2008
New Revision: 691689

URL: http://svn.apache.org/viewvc?rev=691689&view=rev
Log: (empty)

Added:
    incubator/hama/trunk/src/java/org/apache/hama/io/VectorEntry.java
    incubator/hama/trunk/src/java/org/apache/hama/io/VectorMapWritable.java
Modified:
    incubator/hama/trunk/src/java/org/apache/hama/AbstractMatrix.java
    incubator/hama/trunk/src/java/org/apache/hama/DenseMatrix.java
    incubator/hama/trunk/src/java/org/apache/hama/DenseVector.java
    incubator/hama/trunk/src/java/org/apache/hama/Vector.java
    incubator/hama/trunk/src/java/org/apache/hama/algebra/AdditionReduce.java
    incubator/hama/trunk/src/java/org/apache/hama/algebra/MultiplicationMap.java
    incubator/hama/trunk/src/java/org/apache/hama/io/VectorWritable.java
    incubator/hama/trunk/src/test/org/apache/hama/HamaTestCase.java
    incubator/hama/trunk/src/test/org/apache/hama/TestDenseMatrix.java
    incubator/hama/trunk/src/test/org/apache/hama/TestDenseVector.java

Modified: incubator/hama/trunk/src/java/org/apache/hama/AbstractMatrix.java
URL: http://svn.apache.org/viewvc/incubator/hama/trunk/src/java/org/apache/hama/AbstractMatrix.java?rev=691689&r1=691688&r2=691689&view=diff
==============================================================================
--- incubator/hama/trunk/src/java/org/apache/hama/AbstractMatrix.java (original)
+++ incubator/hama/trunk/src/java/org/apache/hama/AbstractMatrix.java Wed Sep  3 10:07:55 2008
@@ -82,8 +82,7 @@
     Cell c;
     double result = -1;
     try {
-      c = table
-          .get(Bytes.toBytes(String.valueOf(i)), Numeric.getColumnIndex(j));
+      c = table.get(Bytes.toBytes(String.valueOf(i)), Numeric.getColumnIndex(j));
       if (c != null) {
         result = Numeric.bytesToDouble(c.getValue());
       }

Modified: incubator/hama/trunk/src/java/org/apache/hama/DenseMatrix.java
URL: http://svn.apache.org/viewvc/incubator/hama/trunk/src/java/org/apache/hama/DenseMatrix.java?rev=691689&r1=691688&r2=691689&view=diff
==============================================================================
--- incubator/hama/trunk/src/java/org/apache/hama/DenseMatrix.java (original)
+++ incubator/hama/trunk/src/java/org/apache/hama/DenseMatrix.java Wed Sep  3 10:07:55 2008
@@ -26,8 +26,6 @@
 import org.apache.hadoop.hbase.HTableDescriptor;
 import org.apache.hadoop.hbase.client.HTable;
 import org.apache.hadoop.hbase.client.Scanner;
-import org.apache.hadoop.hbase.io.Cell;
-import org.apache.hadoop.hbase.io.HbaseMapWritable;
 import org.apache.hadoop.hbase.io.RowResult;
 import org.apache.hadoop.io.IntWritable;
 import org.apache.hadoop.mapred.JobClient;
@@ -36,6 +34,8 @@
 import org.apache.hama.algebra.AdditionReduce;
 import org.apache.hama.algebra.MultiplicationMap;
 import org.apache.hama.algebra.MultiplicationReduce;
+import org.apache.hama.io.VectorEntry;
+import org.apache.hama.io.VectorMapWritable;
 import org.apache.hama.mapred.DenseMap;
 import org.apache.hama.mapred.MatrixReduce;
 import org.apache.hama.util.Numeric;
@@ -164,13 +164,13 @@
     byte[][] c = { columnKey };
     Scanner scan = table.getScanner(c, HConstants.EMPTY_START_ROW);
 
-    HbaseMapWritable<byte[], Cell> trunk = new HbaseMapWritable<byte[], Cell>();
+    VectorMapWritable<Integer, VectorEntry> trunk = new VectorMapWritable<Integer, VectorEntry>();
 
     for (RowResult row : scan) {
-      trunk.put(row.getRow(), row.get(columnKey));
+      trunk.put(Numeric.bytesToInt(row.getRow()), new VectorEntry(row.get(columnKey)));
     }
 
-    return new DenseVector(columnKey, trunk);
+    return new DenseVector(column, trunk);
   }
 
   public Matrix mult(Matrix B) {

Modified: incubator/hama/trunk/src/java/org/apache/hama/DenseVector.java
URL: http://svn.apache.org/viewvc/incubator/hama/trunk/src/java/org/apache/hama/DenseVector.java?rev=691689&r1=691688&r2=691689&view=diff
==============================================================================
--- incubator/hama/trunk/src/java/org/apache/hama/DenseVector.java (original)
+++ incubator/hama/trunk/src/java/org/apache/hama/DenseVector.java Wed Sep  3 10:07:55 2008
@@ -24,8 +24,9 @@
 import java.util.Set;
 
 import org.apache.hadoop.hbase.io.Cell;
-import org.apache.hadoop.hbase.io.HbaseMapWritable;
 import org.apache.hadoop.hbase.io.RowResult;
+import org.apache.hama.io.VectorEntry;
+import org.apache.hama.io.VectorMapWritable;
 import org.apache.hama.io.VectorWritable;
 import org.apache.hama.util.Numeric;
 import org.apache.log4j.Logger;
@@ -34,38 +35,39 @@
   static final Logger LOG = Logger.getLogger(Vector.class);
 
   public DenseVector() {
-    this(null, new HbaseMapWritable<byte[], Cell>());
+    this(-1, new VectorMapWritable<Integer, VectorEntry>());
   }
 
-  public DenseVector(final byte[] row, final HbaseMapWritable<byte[], Cell> m) {
-    byte[] key = row;
+  public DenseVector(final int row,
+      final VectorMapWritable<Integer, VectorEntry> m) {
+    int key = row;
     this.row = key;
-    this.cells = m;
+    this.entries = m;
   }
 
   public DenseVector(int row, RowResult rowResult) {
-    this.cells = new HbaseMapWritable<byte[], Cell>();
-    this.row = Numeric.intToBytes(row);
+    this.entries = new VectorMapWritable<Integer, VectorEntry>();
+    this.row = row;
     for (Map.Entry<byte[], Cell> f : rowResult.entrySet()) {
-      this.cells.put(f.getKey(), f.getValue());
+      this.entries.put(Numeric.getColumnIndex(f.getKey()), new VectorEntry(f
+          .getValue()));
     }
   }
 
   /**
    * Get the row for this Vector
    */
-  public byte[] getRow() {
-    byte[] key = row;
+  public int getRow() {
+    int key = row;
     return key;
   }
 
-  public HbaseMapWritable<byte[], Cell> getCells() {
-    return cells;
+  public VectorMapWritable<Integer, VectorEntry> getCells() {
+    return entries;
   }
 
   public void add(int index, double value) {
     // TODO Auto-generated method stub
-
   }
 
   public Vector add(double alpha, Vector v) {
@@ -77,15 +79,14 @@
     if (this.size() == 0) {
       DenseVector trunk = (DenseVector) v2;
       this.row = trunk.row;
-      this.cells = trunk.cells;
+      this.entries = trunk.entries;
       return this;
     }
 
     for (int i = 0; i < this.size(); i++) {
       double value = (this.get(i) + v2.get(i));
 
-      Cell cValue = new Cell(Numeric.doubleToBytes(value), now());
-      this.cells.put(Numeric.getColumnIndex(i), cValue);
+      this.entries.put(i, new VectorEntry(value));
     }
 
     return this;
@@ -103,17 +104,16 @@
   }
 
   public Vector scale(double alpha) {
-    Set<byte[]> keySet = cells.keySet();
-    Iterator<byte[]> it = keySet.iterator();
+    Set<Integer> keySet = entries.keySet();
+    Iterator<Integer> it = keySet.iterator();
 
     int i = 0;
     while (it.hasNext()) {
-      byte[] key = it.next();
-      double oValue = Numeric.bytesToDouble(this.get(key).getValue());
+      int key = it.next();
+      double oValue = this.get(key);
       double nValue = oValue * alpha;
 
-      Cell cValue = new Cell(Numeric.doubleToBytes(nValue), now());
-      this.cells.put(Numeric.getColumnIndex(i), cValue);
+      this.entries.put(i, new VectorEntry(nValue));
       i++;
     }
 
@@ -121,9 +121,7 @@
   }
 
   public double get(int index) {
-
-    return Numeric.bytesToDouble(cells.get(Numeric.getColumnIndex(index))
-        .getValue());
+    return entries.get(index).getValue();
   }
 
   public double norm(Norm type) {
@@ -138,8 +136,7 @@
   }
 
   public void set(int index, double value) {
-    Cell cValue = new Cell(Numeric.doubleToBytes(value), now());
-    cells.put(Numeric.getColumnIndex(index), cValue);
+    entries.put(index, new VectorEntry(value));
   }
 
   public DenseVector set(Vector v) {
@@ -150,11 +147,11 @@
   public double getNorm1() {
     double sum = 0.0;
 
-    Set<byte[]> keySet = cells.keySet();
-    Iterator<byte[]> it = keySet.iterator();
+    Set<Integer> keySet = entries.keySet();
+    Iterator<Integer> it = keySet.iterator();
 
     while (it.hasNext()) {
-      sum += Numeric.bytesToDouble(this.get(it.next()).getValue());
+      sum += this.get(it.next()).getValue();
     }
 
     return sum;
@@ -163,11 +160,11 @@
   public double getNorm2() {
     double square_sum = 0.0;
 
-    Set<byte[]> keySet = cells.keySet();
-    Iterator<byte[]> it = keySet.iterator();
+    Set<Integer> keySet = entries.keySet();
+    Iterator<Integer> it = keySet.iterator();
 
     while (it.hasNext()) {
-      double value = Numeric.bytesToDouble(this.get(it.next()).getValue());
+      double value = this.get(it.next()).getValue();
       square_sum += value * value;
     }
 

Modified: incubator/hama/trunk/src/java/org/apache/hama/Vector.java
URL: http://svn.apache.org/viewvc/incubator/hama/trunk/src/java/org/apache/hama/Vector.java?rev=691689&r1=691688&r2=691689&view=diff
==============================================================================
--- incubator/hama/trunk/src/java/org/apache/hama/Vector.java (original)
+++ incubator/hama/trunk/src/java/org/apache/hama/Vector.java Wed Sep  3 10:07:55 2008
@@ -21,7 +21,7 @@
 
 import java.util.Iterator;
 
-import org.apache.hadoop.hbase.io.Cell;
+import org.apache.hama.io.VectorEntry;
 
 /**
  * Basic vector interface.
@@ -135,5 +135,5 @@
    * 
    * @return iterator
    */
-  public Iterator<Cell> iterator();
+  public Iterator<VectorEntry> iterator();
 }

Modified: incubator/hama/trunk/src/java/org/apache/hama/algebra/AdditionReduce.java
URL: http://svn.apache.org/viewvc/incubator/hama/trunk/src/java/org/apache/hama/algebra/AdditionReduce.java?rev=691689&r1=691688&r2=691689&view=diff
==============================================================================
--- incubator/hama/trunk/src/java/org/apache/hama/algebra/AdditionReduce.java (original)
+++ incubator/hama/trunk/src/java/org/apache/hama/algebra/AdditionReduce.java Wed Sep  3 10:07:55 2008
@@ -24,11 +24,11 @@
 import java.util.Map;
 
 import org.apache.hadoop.hbase.io.BatchUpdate;
-import org.apache.hadoop.hbase.io.Cell;
 import org.apache.hadoop.io.IntWritable;
 import org.apache.hadoop.mapred.OutputCollector;
 import org.apache.hadoop.mapred.Reporter;
 import org.apache.hama.DenseVector;
+import org.apache.hama.io.VectorEntry;
 import org.apache.hama.mapred.MatrixReduce;
 import org.apache.hama.util.Numeric;
 
@@ -41,8 +41,8 @@
 
     BatchUpdate b = new BatchUpdate(Numeric.intToBytes(key.get()));
     DenseVector vector = values.next();
-    for (Map.Entry<byte[], Cell> f : vector.entrySet()) {
-      b.put(f.getKey(), f.getValue().getValue());
+    for (Map.Entry<Integer, VectorEntry> f : vector.entrySet()) {
+      b.put(Numeric.getColumnIndex(f.getKey()), Numeric.doubleToBytes(f.getValue().getValue()));
     }
 
     output.collect(key, b);

Modified: incubator/hama/trunk/src/java/org/apache/hama/algebra/MultiplicationMap.java
URL: http://svn.apache.org/viewvc/incubator/hama/trunk/src/java/org/apache/hama/algebra/MultiplicationMap.java?rev=691689&r1=691688&r2=691689&view=diff
==============================================================================
--- incubator/hama/trunk/src/java/org/apache/hama/algebra/MultiplicationMap.java (original)
+++ incubator/hama/trunk/src/java/org/apache/hama/algebra/MultiplicationMap.java Wed Sep  3 10:07:55 2008
@@ -22,14 +22,13 @@
 import java.io.IOException;
 import java.util.Iterator;
 
-import org.apache.hadoop.hbase.io.Cell;
 import org.apache.hadoop.io.IntWritable;
 import org.apache.hadoop.mapred.OutputCollector;
 import org.apache.hadoop.mapred.Reporter;
 import org.apache.hama.DenseVector;
 import org.apache.hama.Vector;
+import org.apache.hama.io.VectorEntry;
 import org.apache.hama.mapred.DenseMap;
-import org.apache.hama.util.Numeric;
 import org.apache.log4j.Logger;
 
 public class MultiplicationMap extends DenseMap<IntWritable, DenseVector> {
@@ -40,14 +39,14 @@
       OutputCollector<IntWritable, DenseVector> output, Reporter reporter)
       throws IOException {
 
-    Iterator<Cell> it = value.iterator();
+    Iterator<VectorEntry> it = value.iterator();
     int i = 0;
     while (it.hasNext()) {
-      Cell c = it.next();
+      VectorEntry c = it.next();
       
       Vector v = MATRIX_B.getRow(i);
       
-      double alpha = Numeric.bytesToDouble(c.getValue());
+      double alpha = c.getValue();
       output.collect(key, (DenseVector) v.scale(alpha));
       i++;
     }

Added: incubator/hama/trunk/src/java/org/apache/hama/io/VectorEntry.java
URL: http://svn.apache.org/viewvc/incubator/hama/trunk/src/java/org/apache/hama/io/VectorEntry.java?rev=691689&view=auto
==============================================================================
--- incubator/hama/trunk/src/java/org/apache/hama/io/VectorEntry.java (added)
+++ incubator/hama/trunk/src/java/org/apache/hama/io/VectorEntry.java Wed Sep  3 10:07:55 2008
@@ -0,0 +1,162 @@
+/**
+ * Copyright 2007 The Apache Software Foundation
+ *
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.hama.io;
+
+import java.io.DataInput;
+import java.io.DataOutput;
+import java.io.IOException;
+import java.util.Iterator;
+
+import org.apache.hadoop.hbase.io.Cell;
+import org.apache.hadoop.hbase.util.Bytes;
+import org.apache.hadoop.io.Writable;
+import org.apache.hama.util.Numeric;
+import org.apache.log4j.Logger;
+
+public class VectorEntry implements Writable, Iterable<VectorEntry> {
+  static final Logger LOG = Logger.getLogger(VectorEntry.class);
+  protected byte[][] values;
+  protected long[] timestamps;
+  
+  /** For Writable compatibility */
+  public VectorEntry() {
+    values = null;
+    timestamps = null;
+  }
+
+  public VectorEntry(Cell c) {
+    this.values = new byte[1][];
+    this.values[0] = c.getValue();
+    this.timestamps = new long[1];
+    this.timestamps[0] = c.getTimestamp();
+  }
+
+  public VectorEntry(double value) {
+    this.values = new byte[1][];
+    this.values[0] = Numeric.doubleToBytes(value);
+    this.timestamps = new long[1];
+    this.timestamps[0] = System.currentTimeMillis();
+  }
+  
+  /** @return the current VectorEntry's value */
+  public double getValue() {
+    return Numeric.bytesToDouble(values[0]);
+  }
+
+  /** @return the current VectorEntry's timestamp */
+  public long getTimestamp() {
+    return timestamps[0];
+  }
+
+  /**
+   * Create a new VectorEntry with a given value and timestamp. Used by HStore.
+   * 
+   * @param value
+   * @param timestamp
+   */
+  public VectorEntry(byte[] value, long timestamp) {
+    this.values = new byte[1][];
+    this.values[0] = value;
+    this.timestamps = new long[1];
+    this.timestamps[0] = timestamp;
+  }
+
+  /** {@inheritDoc} */
+  @Override
+  public String toString() {
+    if (this.values.length == 1) {
+      return "timestamp=" + this.timestamps[0] + ", value="
+          + Bytes.toString(this.values[0]);
+    }
+    StringBuilder s = new StringBuilder("{ ");
+    for (int i = 0; i < this.values.length; i++) {
+      if (i > 0) {
+        s.append(", ");
+      }
+      s.append("[timestamp=");
+      s.append(timestamps[i]);
+      s.append(", value=");
+      s.append(Bytes.toString(values[i]));
+      s.append("]");
+    }
+    s.append(" }");
+    return s.toString();
+  }
+
+  //
+  // Writable
+  //
+
+  /** {@inheritDoc} */
+  public void readFields(final DataInput in) throws IOException {
+    int nvalues = in.readInt();
+    this.timestamps = new long[nvalues];
+    this.values = new byte[nvalues][];
+    for (int i = 0; i < nvalues; i++) {
+      this.timestamps[i] = in.readLong();
+    }
+    for (int i = 0; i < nvalues; i++) {
+      this.values[i] = Bytes.readByteArray(in);
+    }
+  }
+
+  /** {@inheritDoc} */
+  public void write(final DataOutput out) throws IOException {
+    out.writeInt(this.values.length);
+    for (int i = 0; i < this.timestamps.length; i++) {
+      out.writeLong(this.timestamps[i]);
+    }
+    for (int i = 0; i < this.values.length; i++) {
+      Bytes.writeByteArray(out, this.values[i]);
+    }
+  }
+
+  //
+  // Iterable
+  //
+
+  /** {@inheritDoc} */
+  public Iterator<VectorEntry> iterator() {
+    return new VectorEntryIterator();
+  }
+
+  private class VectorEntryIterator implements Iterator<VectorEntry> {
+    private int currentValue = -1;
+
+    VectorEntryIterator() {
+    }
+
+    /** {@inheritDoc} */
+    public boolean hasNext() {
+      return currentValue < values.length;
+    }
+
+    /** {@inheritDoc} */
+    public VectorEntry next() {
+      currentValue += 1;
+      return new VectorEntry(values[currentValue], timestamps[currentValue]);
+    }
+
+    /** {@inheritDoc} */
+    public void remove() throws UnsupportedOperationException {
+      throw new UnsupportedOperationException("remove is not supported");
+    }
+  }
+}

Added: incubator/hama/trunk/src/java/org/apache/hama/io/VectorMapWritable.java
URL: http://svn.apache.org/viewvc/incubator/hama/trunk/src/java/org/apache/hama/io/VectorMapWritable.java?rev=691689&view=auto
==============================================================================
--- incubator/hama/trunk/src/java/org/apache/hama/io/VectorMapWritable.java (added)
+++ incubator/hama/trunk/src/java/org/apache/hama/io/VectorMapWritable.java Wed Sep  3 10:07:55 2008
@@ -0,0 +1,190 @@
+/**
+ * Copyright 2007 The Apache Software Foundation
+ *
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.hama.io;
+
+import java.io.DataInput;
+import java.io.DataOutput;
+import java.io.IOException;
+import java.util.Collection;
+import java.util.HashMap;
+import java.util.Map;
+import java.util.Set;
+import java.util.TreeMap;
+import java.util.concurrent.atomic.AtomicReference;
+
+import org.apache.hadoop.conf.Configurable;
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.hbase.HStoreKey;
+import org.apache.hadoop.hbase.io.ImmutableBytesWritable;
+import org.apache.hadoop.hbase.util.Bytes;
+import org.apache.hadoop.io.Text;
+import org.apache.hadoop.io.Writable;
+import org.apache.hadoop.util.ReflectionUtils;
+import org.apache.hama.util.Numeric;
+
+public class VectorMapWritable<K, V> implements Map<Integer, V>, Writable,
+    Configurable {
+  private AtomicReference<Configuration> conf = new AtomicReference<Configuration>();
+
+  // Static maps of code to class and vice versa. Includes types used in hbase
+  // only.
+  static final Map<Byte, Class<?>> CODE_TO_CLASS = new HashMap<Byte, Class<?>>();
+  static final Map<Class<?>, Byte> CLASS_TO_CODE = new HashMap<Class<?>, Byte>();
+
+  static {
+    byte code = 0;
+    addToMap(HStoreKey.class, code++);
+    addToMap(ImmutableBytesWritable.class, code++);
+    addToMap(Text.class, code++);
+    addToMap(VectorEntry.class, code++);
+    addToMap(byte[].class, code++);
+  }
+
+  @SuppressWarnings("boxing")
+  private static void addToMap(final Class<?> clazz, final byte code) {
+    CLASS_TO_CODE.put(clazz, code);
+    CODE_TO_CLASS.put(code, clazz);
+  }
+
+  private Map<Integer, V> instance = new TreeMap<Integer, V>();
+
+  /** @return the conf */
+  public Configuration getConf() {
+    return conf.get();
+  }
+
+  /** @param conf the conf to set */
+  public void setConf(Configuration conf) {
+    this.conf.set(conf);
+  }
+
+  /** {@inheritDoc} */
+  public void clear() {
+    instance.clear();
+  }
+
+  /** {@inheritDoc} */
+  public boolean containsKey(Object key) {
+    return instance.containsKey(key);
+  }
+
+  /** {@inheritDoc} */
+  public boolean containsValue(Object value) {
+    return instance.containsValue(value);
+  }
+
+  /** {@inheritDoc} */
+  public Set<Entry<Integer, V>> entrySet() {
+    return instance.entrySet();
+  }
+
+  /** {@inheritDoc} */
+  public V get(Object key) {
+    return instance.get(key);
+  }
+
+  /** {@inheritDoc} */
+  public boolean isEmpty() {
+    return instance.isEmpty();
+  }
+
+  /** {@inheritDoc} */
+  public Set<Integer> keySet() {
+    return instance.keySet();
+  }
+
+  /** {@inheritDoc} */
+  public int size() {
+    return instance.size();
+  }
+
+  /** {@inheritDoc} */
+  public Collection<V> values() {
+    return instance.values();
+  }
+
+  // Writable
+
+  /** @return the Class class for the specified id */
+  @SuppressWarnings( { "unchecked", "boxing" })
+  protected Class<?> getClass(byte id) {
+    return CODE_TO_CLASS.get(id);
+  }
+
+  /** @return the id for the specified Class */
+  @SuppressWarnings( { "unchecked", "boxing" })
+  protected byte getId(Class<?> clazz) {
+    Byte b = CLASS_TO_CODE.get(clazz);
+    if (b == null) {
+      throw new NullPointerException("Nothing for : " + clazz);
+    }
+    return b;
+  }
+
+  @Override
+  public String toString() {
+    return this.instance.toString();
+  }
+
+  /** {@inheritDoc} */
+  public void write(DataOutput out) throws IOException {
+    // Write out the number of entries in the map
+    out.writeInt(this.instance.size());
+
+    // Then write out each key/value pair
+    for (Map.Entry<Integer, V> e : instance.entrySet()) {
+      Bytes.writeByteArray(out, Numeric.getColumnIndex(e.getKey()));
+      out.writeByte(getId(e.getValue().getClass()));
+      ((Writable) e.getValue()).write(out);
+    }
+  }
+
+  /** {@inheritDoc} */
+  public void readFields(DataInput in) throws IOException {
+    // First clear the map. Otherwise we will just accumulate
+    // entries every time this method is called.
+    this.instance.clear();
+
+    // Read the number of entries in the map
+    int entries = in.readInt();
+
+    // Then read each key/value pair
+    for (int i = 0; i < entries; i++) {
+      byte[] key = Bytes.readByteArray(in);
+      Writable value = (Writable) ReflectionUtils.newInstance(getClass(in
+          .readByte()), getConf());
+      value.readFields(in);
+      V v = (V) value;
+      this.instance.put(Numeric.getColumnIndex(key), v);
+    }
+  }
+
+  public void putAll(Map<? extends Integer, ? extends V> m) {
+    this.instance.putAll(m);
+  }
+
+  public V remove(Object key) {
+    return this.instance.remove(key);
+  }
+
+  public V put(Integer key, V value) {
+    return this.instance.put(key, value);
+  }
+}

Modified: incubator/hama/trunk/src/java/org/apache/hama/io/VectorWritable.java
URL: http://svn.apache.org/viewvc/incubator/hama/trunk/src/java/org/apache/hama/io/VectorWritable.java?rev=691689&r1=691688&r2=691689&view=diff
==============================================================================
--- incubator/hama/trunk/src/java/org/apache/hama/io/VectorWritable.java (original)
+++ incubator/hama/trunk/src/java/org/apache/hama/io/VectorWritable.java Wed Sep  3 10:07:55 2008
@@ -31,33 +31,32 @@
 import java.util.TreeSet;
 
 import org.apache.hadoop.hbase.HConstants;
-import org.apache.hadoop.hbase.io.Cell;
-import org.apache.hadoop.hbase.io.HbaseMapWritable;
 import org.apache.hadoop.hbase.util.Bytes;
 import org.apache.hadoop.hbase.util.Writables;
 import org.apache.hadoop.io.Writable;
 import org.apache.hama.Vector;
 import org.apache.hama.util.Numeric;
 
-public class VectorWritable implements Writable, Map<byte[], Cell> {
+public abstract class VectorWritable implements Writable,
+    Map<Integer, VectorEntry> {
 
-  public byte[] row;
-  public HbaseMapWritable<byte[], Cell> cells;
+  public Integer row;
+  public VectorMapWritable<Integer, VectorEntry> entries;
 
-  public Cell put(byte[] key, Cell value) {
+  public VectorEntry put(Integer key, VectorEntry value) {
     throw new UnsupportedOperationException("VectorWritable is read-only!");
   }
 
-  public Cell get(Object key) {
-    return this.cells.get(key);
+  public VectorEntry get(Object key) {
+    return this.entries.get(key);
   }
 
-  public Cell remove(Object key) {
+  public VectorEntry remove(Object key) {
     throw new UnsupportedOperationException("VectorWritable is read-only!");
   }
 
   public boolean containsKey(Object key) {
-    return cells.containsKey(key);
+    return entries.containsKey(key);
   }
 
   public boolean containsValue(Object value) {
@@ -65,62 +64,62 @@
   }
 
   public boolean isEmpty() {
-    return cells.isEmpty();
+    return entries.isEmpty();
   }
 
   public void clear() {
     throw new UnsupportedOperationException("VectorDatum is read-only!");
   }
 
-  public Set<byte[]> keySet() {
-    Set<byte[]> result = new TreeSet<byte[]>(Bytes.BYTES_COMPARATOR);
-    for (byte[] w : cells.keySet()) {
+  public Set<Integer> keySet() {
+    Set<Integer> result = new TreeSet<Integer>();
+    for (Integer w : entries.keySet()) {
       result.add(w);
     }
     return result;
   }
 
-  public Set<Map.Entry<byte[], Cell>> entrySet() {
-    return Collections.unmodifiableSet(this.cells.entrySet());
+  public Set<Map.Entry<Integer, VectorEntry>> entrySet() {
+    return Collections.unmodifiableSet(this.entries.entrySet());
   }
 
-  public Collection<Cell> values() {
-    ArrayList<Cell> result = new ArrayList<Cell>();
-    for (Writable w : cells.values()) {
-      result.add((Cell) w);
+  public Collection<VectorEntry> values() {
+    ArrayList<VectorEntry> result = new ArrayList<VectorEntry>();
+    for (Writable w : entries.values()) {
+      result.add((VectorEntry) w);
     }
     return result;
   }
 
   public void readFields(final DataInput in) throws IOException {
-    this.row = Bytes.readByteArray(in);
-    this.cells.readFields(in);
+    this.row = Numeric.bytesToInt(Bytes.readByteArray(in));
+    this.entries.readFields(in);
   }
 
   public void write(final DataOutput out) throws IOException {
-    Bytes.writeByteArray(out, this.row);
-    this.cells.write(out);
+    Bytes.writeByteArray(out, Numeric.intToBytes(this.row));
+    this.entries.write(out);
   }
 
-  public VectorWritable addition(byte[] bs, Vector v2) {
+  public VectorWritable addition(Integer bs, Vector v2) {
     throw new UnsupportedOperationException("Not implemented yet");
   }
 
-  public void putAll(Map<? extends byte[], ? extends Cell> m) {
+  public void putAll(Map<? extends Integer, ? extends VectorEntry> m) {
     throw new UnsupportedOperationException("Not implemented yet");
   }
 
   /**
-   * Get the Cell that corresponds to column
+   * Get the VectorEntry that corresponds to column
    */
-  public Cell get(byte[] column) {
-    return this.cells.get(column);
+  public VectorEntry get(Integer column) {
+    return this.entries.get(column);
   }
 
   /**
-   * Get the Cell that corresponds to column, using a String key
+   * Get the VectorEntry that corresponds to column, using a String key
    */
-  public Cell get(String key) {
+  public VectorEntry get(String key) {
     return get(Bytes.toBytes(key));
   }
 
@@ -128,33 +127,33 @@
    * Get the double value without timestamp
    */
   public double get(int key) {
-    return Numeric.bytesToDouble(get(Numeric.intToBytes(key)).getValue());
+    return this.get(Numeric.intToBytes(key)).getValue();
   }
 
   public int size() {
-    return this.cells.size();
+    return this.entries.size();
   }
 
   @Override
   public String toString() {
     StringBuilder sb = new StringBuilder();
     sb.append("row=");
-    sb.append(Bytes.toString(this.row));
+    sb.append(String.valueOf(this.row));
     sb.append(", cells={");
     boolean moreThanOne = false;
-    for (Map.Entry<byte[], Cell> e : this.cells.entrySet()) {
+    for (Map.Entry<Integer, VectorEntry> e : this.entries.entrySet()) {
       if (moreThanOne) {
         sb.append(", ");
       } else {
         moreThanOne = true;
       }
       sb.append("(column=");
-      sb.append(Bytes.toString(e.getKey()));
+      sb.append(String.valueOf(e.getKey()));
       sb.append(", timestamp=");
       sb.append(Long.toString(e.getValue().getTimestamp()));
       sb.append(", value=");
-      byte[] value = e.getValue().getValue();
-      if (Bytes.equals(e.getKey(), HConstants.COL_REGIONINFO)) {
+      byte[] value = Numeric.doubleToBytes(e.getValue().getValue());
+      if (Bytes.equals(Numeric.intToBytes(e.getKey()), HConstants.COL_REGIONINFO)) {
         try {
           sb.append(Writables.getHRegionInfo(value).toString());
         } catch (IOException ioe) {
@@ -174,8 +173,8 @@
    * 
    * @return iterator
    */
-  public Iterator<Cell> iterator() {
-    return cells.values().iterator();
+  public Iterator<VectorEntry> iterator() {
+    return entries.values().iterator();
   }
 
   /**
@@ -183,17 +182,17 @@
    * The inner class for an entry of row.
    * 
    */
-  public static class Entries implements Map.Entry<byte[], Cell> {
+  public static class Entries implements Map.Entry<byte[], VectorEntry> {
 
     private final byte[] column;
-    private final Cell cell;
+    private final VectorEntry entry;
 
-    Entries(byte[] column, Cell cell) {
+    Entries(byte[] column, VectorEntry entry) {
       this.column = column;
-      this.cell = cell;
+      this.entry = entry;
     }
 
-    public Cell setValue(Cell c) {
+    public VectorEntry setValue(VectorEntry c) {
       throw new UnsupportedOperationException("VectorWritable is read-only!");
     }
 
@@ -202,12 +201,8 @@
       return key;
     }
 
-    public Cell getValue() {
-      return cell;
+    public VectorEntry getValue() {
+      return entry;
     }
   }
-  
-  public long now() {
-    return System.currentTimeMillis();
-  }
 }

Modified: incubator/hama/trunk/src/test/org/apache/hama/HamaTestCase.java
URL: http://svn.apache.org/viewvc/incubator/hama/trunk/src/test/org/apache/hama/HamaTestCase.java?rev=691689&r1=691688&r2=691689&view=diff
==============================================================================
--- incubator/hama/trunk/src/test/org/apache/hama/HamaTestCase.java (original)
+++ incubator/hama/trunk/src/test/org/apache/hama/HamaTestCase.java Wed Sep  3 10:07:55 2008
@@ -26,7 +26,7 @@
  */
 public class HamaTestCase extends HBaseClusterTestCase {
   protected int SIZE = 5;
-  protected HamaConfiguration conf = new HamaConfiguration();
+  protected final HamaConfiguration conf = new HamaConfiguration();
   
   /** constructor */
   public HamaTestCase() {

Modified: incubator/hama/trunk/src/test/org/apache/hama/TestDenseMatrix.java
URL: http://svn.apache.org/viewvc/incubator/hama/trunk/src/test/org/apache/hama/TestDenseMatrix.java?rev=691689&r1=691688&r2=691689&view=diff
==============================================================================
--- incubator/hama/trunk/src/test/org/apache/hama/TestDenseMatrix.java (original)
+++ incubator/hama/trunk/src/test/org/apache/hama/TestDenseMatrix.java Wed Sep  3 10:07:55 2008
@@ -22,13 +22,15 @@
 import java.io.IOException;
 import java.util.Iterator;
 
-import org.apache.hadoop.hbase.io.Cell;
-import org.apache.hama.util.Numeric;
+import org.apache.hama.io.VectorEntry;
+import org.apache.log4j.Logger;
 
 /**
  * Matrix test
  */
 public class TestDenseMatrix extends HamaTestCase {
+  static final Logger LOG = Logger.getLogger(TestDenseMatrix.class);
+  
   private Matrix m1;
   private Matrix m2;
   
@@ -46,10 +48,10 @@
    */
   public void testGetColumn() throws IOException {
     Vector v = m1.getColumn(0);
-    Iterator<Cell> it = v.iterator();
+    Iterator<VectorEntry> it = v.iterator();
     int x = 0;
     while (it.hasNext()) {
-      assertEquals(m1.get(x, 0), Numeric.bytesToDouble(it.next().getValue()));
+      assertEquals(m1.get(x, 0), it.next().getValue());
       x++;
     }
   }
@@ -94,6 +96,7 @@
 
     for (int i = 0; i < 2; i++) {
       for (int j = 0; j < 2; j++) {
+        LOG.info("result: " + result.get(i, j) + ", C: " + C[i][j]);
         assertEquals(result.get(i, j), C[i][j]);
       }
     }

Modified: incubator/hama/trunk/src/test/org/apache/hama/TestDenseVector.java
URL: http://svn.apache.org/viewvc/incubator/hama/trunk/src/test/org/apache/hama/TestDenseVector.java?rev=691689&r1=691688&r2=691689&view=diff
==============================================================================
--- incubator/hama/trunk/src/test/org/apache/hama/TestDenseVector.java (original)
+++ incubator/hama/trunk/src/test/org/apache/hama/TestDenseVector.java Wed Sep  3 10:07:55 2008
@@ -21,14 +21,13 @@
 
 import java.util.Iterator;
 
-import org.apache.hadoop.hbase.io.Cell;
-import org.apache.hama.util.Numeric;
+import org.apache.hama.io.VectorEntry;
 
 public class TestDenseVector extends HamaTestCase {
   private static final double cosine = 0.6978227007909176;
   private static final double norm1 = 12.0;
   private static final double norm2 = 6.782329983125268;
-  private double[][] values = { { 2, 5, 1, 4 }, { 4, 1, 3, 3 } };
+  private static double[][] values = { { 2, 5, 1, 4 }, { 4, 1, 3, 3 } };
   private Matrix m1;
   private Vector v1;
   private Vector v2;
@@ -90,10 +89,10 @@
    */
   public void testIterator() {
     int i = 0;
-    Iterator<Cell> it = v1.iterator();
+    Iterator<VectorEntry> it = v1.iterator();
     while (it.hasNext()) {
-      Cell c = it.next();
-      assertEquals(Numeric.bytesToDouble(c.getValue()), values[0][i]);
+      VectorEntry c = it.next();
+      assertEquals(c.getValue(), values[0][i]);
       i++;
     }
   }