Author: edwardyoon
Date: Thu Mar 12 07:40:36 2015
New Revision: 1666087

URL: http://svn.apache.org/r1666087
Log:
Reverting. Sorry for my miss-commit

Modified:
    
hama/trunk/commons/src/main/java/org/apache/hama/commons/math/DenseDoubleMatrix.java
    
hama/trunk/commons/src/main/java/org/apache/hama/commons/math/DenseDoubleVector.java
    hama/trunk/core/pom.xml
    hama/trunk/core/src/main/java/org/apache/hama/bsp/BSPMessageBundle.java
    hama/trunk/core/src/test/java/org/apache/hama/util/TestKryoSerializer.java
    hama/trunk/graph/pom.xml
    hama/trunk/graph/src/main/java/org/apache/hama/graph/GraphJobMessage.java
    hama/trunk/graph/src/main/java/org/apache/hama/graph/GraphJobRunner.java
    
hama/trunk/graph/src/main/java/org/apache/hama/graph/OutgoingVertexMessageManager.java
    
hama/trunk/ml/src/test/java/org/apache/hama/ml/recommendation/TestOnlineCF.java
    hama/trunk/pom.xml

Modified: 
hama/trunk/commons/src/main/java/org/apache/hama/commons/math/DenseDoubleMatrix.java
URL: 
http://svn.apache.org/viewvc/hama/trunk/commons/src/main/java/org/apache/hama/commons/math/DenseDoubleMatrix.java?rev=1666087&r1=1666086&r2=1666087&view=diff
==============================================================================
--- 
hama/trunk/commons/src/main/java/org/apache/hama/commons/math/DenseDoubleMatrix.java
 (original)
+++ 
hama/trunk/commons/src/main/java/org/apache/hama/commons/math/DenseDoubleMatrix.java
 Thu Mar 12 07:40:36 2015
@@ -29,12 +29,10 @@ import com.google.common.base.Preconditi
  */
 public final class DenseDoubleMatrix implements DoubleMatrix {
 
-  protected double[][] matrix;
-  protected int numRows;
-  protected int numColumns;
+  protected final double[][] matrix;
+  protected final int numRows;
+  protected final int numColumns;
 
-  public DenseDoubleMatrix() {}
-  
   /**
    * Creates a new empty matrix from the rows and columns.
    * 

Modified: 
hama/trunk/commons/src/main/java/org/apache/hama/commons/math/DenseDoubleVector.java
URL: 
http://svn.apache.org/viewvc/hama/trunk/commons/src/main/java/org/apache/hama/commons/math/DenseDoubleVector.java?rev=1666087&r1=1666086&r2=1666087&view=diff
==============================================================================
--- 
hama/trunk/commons/src/main/java/org/apache/hama/commons/math/DenseDoubleVector.java
 (original)
+++ 
hama/trunk/commons/src/main/java/org/apache/hama/commons/math/DenseDoubleVector.java
 Thu Mar 12 07:40:36 2015
@@ -33,10 +33,8 @@ import com.google.common.collect.Abstrac
  */
 public final class DenseDoubleVector implements DoubleVector {
 
-  private double[] vector;
+  private final double[] vector;
 
-  public DenseDoubleVector() {}
-  
   /**
    * Creates a new vector with the given length.
    */

Modified: hama/trunk/core/pom.xml
URL: 
http://svn.apache.org/viewvc/hama/trunk/core/pom.xml?rev=1666087&r1=1666086&r2=1666087&view=diff
==============================================================================
--- hama/trunk/core/pom.xml (original)
+++ hama/trunk/core/pom.xml Thu Mar 12 07:40:36 2015
@@ -141,9 +141,14 @@
       <artifactId>zookeeper</artifactId>
     </dependency>
     <dependency>
-      <groupId>com.esotericsoftware</groupId>
-      <artifactId>kryo</artifactId>
-      <version>${kryo.version}</version>
+      <groupId>org.apache.directmemory</groupId>
+      <artifactId>directmemory-cache</artifactId>
+      <version>0.2</version>
+    </dependency>
+    <dependency>
+      <groupId>org.apache.directmemory</groupId>
+      <artifactId>directmemory-kryo</artifactId>
+      <version>0.2</version>
     </dependency>
     <dependency>
       <groupId>io.netty</groupId>

Modified: 
hama/trunk/core/src/main/java/org/apache/hama/bsp/BSPMessageBundle.java
URL: 
http://svn.apache.org/viewvc/hama/trunk/core/src/main/java/org/apache/hama/bsp/BSPMessageBundle.java?rev=1666087&r1=1666086&r2=1666087&view=diff
==============================================================================
--- hama/trunk/core/src/main/java/org/apache/hama/bsp/BSPMessageBundle.java 
(original)
+++ hama/trunk/core/src/main/java/org/apache/hama/bsp/BSPMessageBundle.java Thu 
Mar 12 07:40:36 2015
@@ -20,17 +20,16 @@ package org.apache.hama.bsp;
 import java.io.ByteArrayInputStream;
 import java.io.ByteArrayOutputStream;
 import java.io.DataInput;
+import java.io.DataInputStream;
 import java.io.DataOutput;
+import java.io.DataOutputStream;
 import java.io.IOException;
 import java.util.Iterator;
 
 import org.apache.commons.logging.Log;
 import org.apache.commons.logging.LogFactory;
 import org.apache.hadoop.io.Writable;
-
-import com.esotericsoftware.kryo.Kryo;
-import com.esotericsoftware.kryo.io.Input;
-import com.esotericsoftware.kryo.io.Output;
+import org.apache.hadoop.util.ReflectionUtils;
 
 /**
  * BSPMessageBundle stores a group of messages so that they can be sent in 
batch
@@ -45,9 +44,8 @@ public class BSPMessageBundle<M extends
   private String className = null;
   private int bundleSize = 0;
 
-  private Kryo kryo = new Kryo();
-  private ByteArrayOutputStream outputStream = new ByteArrayOutputStream();
-  private Output output = new Output(outputStream, 4096);
+  private final ByteArrayOutputStream byteBuffer = new ByteArrayOutputStream();
+  private final DataOutputStream bufferDos = new DataOutputStream(byteBuffer);
 
   public BSPMessageBundle() {
     bundleSize = 0;
@@ -59,35 +57,42 @@ public class BSPMessageBundle<M extends
    * @param message BSPMessage to add.
    */
   public void addMessage(M message) {
-    if (className == null) {
-      className = message.getClass().getName();
-      kryo.register(message.getClass());
-    }
-
-    kryo.writeObject(output, message);
-    output.flush();
+    try {
+      if (className == null) {
+        className = message.getClass().getName();
+      }
 
-    bundleSize++;
+      message.write(bufferDos);
+      bundleSize++;
+    } catch (IOException e) {
+      LOG.error(e);
+    }
   }
 
   public byte[] getBuffer() {
-    return outputStream.toByteArray();
+    return byteBuffer.toByteArray();
   }
 
+  private ByteArrayInputStream bis = null;
+  private DataInputStream dis = null;
+
   public Iterator<M> iterator() {
+    bis = new ByteArrayInputStream(byteBuffer.toByteArray());
+    dis = new DataInputStream(bis);
 
     Iterator<M> it = new Iterator<M>() {
-      ByteArrayInputStream bis = new 
ByteArrayInputStream(outputStream.toByteArray());
-      Input in = new Input(bis, 4096);
-      
       Class<M> clazz = null;
-      int counter = 0;
+      M msg;
 
       @Override
       public boolean hasNext() {
-        if ((bundleSize - counter) > 0) {
-          return true;
-        } else {
+        try {
+          if (dis.available() > 0) {
+            return true;
+          } else {
+            return false;
+          }
+        } catch (IOException e) {
           return false;
         }
       }
@@ -99,13 +104,17 @@ public class BSPMessageBundle<M extends
           if (clazz == null) {
             clazz = (Class<M>) Class.forName(className);
           }
+
+          msg = ReflectionUtils.newInstance(clazz, null);
+          msg.readFields(dis);
+
+        } catch (IOException ie) {
+          LOG.error(ie);
         } catch (ClassNotFoundException ce) {
           LOG.error("Class was not found.", ce);
         }
 
-        counter++;
-
-        return kryo.readObject(in, clazz);
+        return msg;
       }
 
       @Override
@@ -125,30 +134,29 @@ public class BSPMessageBundle<M extends
    * @throws IOException
    */
   public long getLength() {
-    return outputStream.size();
+    return byteBuffer.size();
   }
 
   @Override
   public void write(DataOutput out) throws IOException {
     out.writeInt(bundleSize);
-    
     if (bundleSize > 0) {
       out.writeUTF(className);
-      out.writeInt(outputStream.size());
-      out.write(outputStream.toByteArray());
+      out.writeInt(byteBuffer.size());
+      out.write(byteBuffer.toByteArray());
     }
   }
 
   @Override
   public void readFields(DataInput in) throws IOException {
-    bundleSize = in.readInt();
+    this.bundleSize = in.readInt();
 
-    if (bundleSize > 0) {
+    if (this.bundleSize > 0) {
       className = in.readUTF();
       int bytesLength = in.readInt();
       byte[] temp = new byte[bytesLength];
       in.readFully(temp);
-      outputStream.write(temp);
+      bufferDos.write(temp);
     }
   }
 

Modified: 
hama/trunk/core/src/test/java/org/apache/hama/util/TestKryoSerializer.java
URL: 
http://svn.apache.org/viewvc/hama/trunk/core/src/test/java/org/apache/hama/util/TestKryoSerializer.java?rev=1666087&r1=1666086&r2=1666087&view=diff
==============================================================================
--- hama/trunk/core/src/test/java/org/apache/hama/util/TestKryoSerializer.java 
(original)
+++ hama/trunk/core/src/test/java/org/apache/hama/util/TestKryoSerializer.java 
Thu Mar 12 07:40:36 2015
@@ -19,39 +19,32 @@ package org.apache.hama.util;
 
 import java.io.ByteArrayInputStream;
 import java.io.ByteArrayOutputStream;
+import java.io.DataInputStream;
+import java.io.DataOutputStream;
 
 import junit.framework.TestCase;
 
 import org.apache.hadoop.io.DoubleWritable;
 
-import com.esotericsoftware.kryo.Kryo;
-import com.esotericsoftware.kryo.io.Input;
-import com.esotericsoftware.kryo.io.Output;
-
 public class TestKryoSerializer extends TestCase {
 
-  public void testSerialization() throws Exception {
-    Kryo kryo = new Kryo();
-    kryo.register(DoubleWritable.class);
-
-    ByteArrayOutputStream outputStream = new ByteArrayOutputStream();
-    Output out = new Output(outputStream, 4096);
+  ByteArrayOutputStream mbos = null;
+  DataOutputStream mdos = null;
+  ByteArrayInputStream mbis = null;
+  DataInputStream mdis = null;
 
-    for (int i = 0; i < 10; i++) {
-      DoubleWritable a = new DoubleWritable(i + 0.123);
-      kryo.writeClassAndObject(out, a);
-      out.flush();
-    }
-
-    System.out.println(outputStream.size());
+  public void testSerialization() throws Exception {
+    KryoSerializer k = new KryoSerializer(DoubleWritable.class);
     
-    ByteArrayInputStream bin = new 
ByteArrayInputStream(outputStream.toByteArray());
-    Input in = new Input(bin, 4096);
-
-    for (int i = 0; i < 10; i++) {
-      DoubleWritable b = (DoubleWritable) kryo.readClassAndObject(in);
-      System.out.println(bin.available() + ", " + b);
+    long startTime = System.currentTimeMillis();
+    for (int i = 0; i < 10000000; i++) {
+      DoubleWritable x = new DoubleWritable(i + 0.2);
+      byte[] bytes = k.serialize(x);
+      DoubleWritable y = (DoubleWritable) k.deserialize(bytes);
+      assertEquals(x, y);
     }
+    System.out.println("Finished in "
+        + (System.currentTimeMillis() - startTime) / 1000.0 + " seconds");
   }
 
 }

Modified: hama/trunk/graph/pom.xml
URL: 
http://svn.apache.org/viewvc/hama/trunk/graph/pom.xml?rev=1666087&r1=1666086&r2=1666087&view=diff
==============================================================================
--- hama/trunk/graph/pom.xml (original)
+++ hama/trunk/graph/pom.xml Thu Mar 12 07:40:36 2015
@@ -48,16 +48,6 @@
       <type>test-jar</type>
       <scope>test</scope>
     </dependency>
-    <dependency>
-      <groupId>org.apache.directmemory</groupId>
-      <artifactId>directmemory-cache</artifactId>
-      <version>0.2</version>
-    </dependency>
-    <dependency>
-      <groupId>org.apache.directmemory</groupId>
-      <artifactId>directmemory-kryo</artifactId>
-      <version>0.2</version>
-    </dependency>
   </dependencies>
   <build>
     <finalName>hama-graph-${project.version}</finalName>

Modified: 
hama/trunk/graph/src/main/java/org/apache/hama/graph/GraphJobMessage.java
URL: 
http://svn.apache.org/viewvc/hama/trunk/graph/src/main/java/org/apache/hama/graph/GraphJobMessage.java?rev=1666087&r1=1666086&r2=1666087&view=diff
==============================================================================
--- hama/trunk/graph/src/main/java/org/apache/hama/graph/GraphJobMessage.java 
(original)
+++ hama/trunk/graph/src/main/java/org/apache/hama/graph/GraphJobMessage.java 
Thu Mar 12 07:40:36 2015
@@ -17,14 +17,11 @@
  */
 package org.apache.hama.graph;
 
-import java.io.ByteArrayInputStream;
 import java.io.ByteArrayOutputStream;
 import java.io.DataInput;
-import java.io.DataInputStream;
 import java.io.DataOutput;
 import java.io.DataOutputStream;
 import java.io.IOException;
-import java.util.Iterator;
 import java.util.List;
 
 import org.apache.hadoop.io.DataInputBuffer;
@@ -57,7 +54,8 @@ public final class GraphJobMessage imple
 
   private int numOfValues = 0;
 
-  private final ByteArrayOutputStream bytesStream = new 
ByteArrayOutputStream();
+  private final ByteArrayOutputStream byteBuffer = new ByteArrayOutputStream();
+  private final DataOutputStream bufferDos = new DataOutputStream(byteBuffer);
 
   static {
     if (comparator == null) {
@@ -94,7 +92,7 @@ public final class GraphJobMessage imple
     this.flag = VERTEX_FLAG;
     this.vertexId = vertexID;
     try {
-      this.bytesStream.write(valuesBytes);
+      this.bufferDos.write(valuesBytes);
     } catch (IOException e) {
       e.printStackTrace();
     }
@@ -111,23 +109,25 @@ public final class GraphJobMessage imple
   }
 
   public byte[] getValuesBytes() {
-    return bytesStream.toByteArray();
+    return byteBuffer.toByteArray();
   }
 
   public void addValuesBytes(byte[] values, int numOfValues) {
     try {
-      bytesStream.write(values);
+      bufferDos.write(values);
       this.numOfValues += numOfValues;
     } catch (IOException e) {
+      // TODO Auto-generated catch block
       e.printStackTrace();
     }
   }
 
   public void add(Writable value) {
     try {
-      value.write(new DataOutputStream(bytesStream));
+      value.write(bufferDos);
       numOfValues++;
     } catch (IOException e) {
+      // TODO Auto-generated catch block
       e.printStackTrace();
     }
   }
@@ -155,8 +155,8 @@ public final class GraphJobMessage imple
       vertexId.write(out);
 
       out.writeInt(numOfValues);
-      out.writeInt(bytesStream.size());
-      out.write(bytesStream.toByteArray());
+      out.writeInt(byteBuffer.size());
+      out.write(byteBuffer.toByteArray());
     } else if (isMapMessage()) {
       map.write(out);
     } else if (isVerticesSizeMessage()) {
@@ -199,7 +199,7 @@ public final class GraphJobMessage imple
       int bytesLength = in.readInt();
       byte[] temp = new byte[bytesLength];
       in.readFully(temp);
-      bytesStream.write(temp);
+      bufferDos.write(temp);
     } else if (isMapMessage()) {
       map = new MapWritable();
       map.readFields(in);
@@ -304,41 +304,4 @@ public final class GraphJobMessage imple
     }
   }
 
-
-  public Iterable<Writable> getIterableMessages() {
-
-    return new Iterable<Writable>() {
-      @Override
-      public Iterator<Writable> iterator() {
-        return new Iterator<Writable>() {
-          ByteArrayInputStream bis = new 
ByteArrayInputStream(bytesStream.toByteArray());
-          DataInputStream dis = new DataInputStream(bis);
-          int index = 0;
-
-          @Override
-          public boolean hasNext() {
-            return (index < numOfValues) ? true : false;
-          }
-
-          @Override
-          public Writable next() {
-            Writable v = GraphJobRunner.createVertexValue();
-            try {
-              v.readFields(dis);
-            } catch (IOException e) {
-              e.printStackTrace();
-            }
-            index++;
-            return v;
-          }
-
-          @Override
-          public void remove() {
-          }
-        };
-      }
-    };
-  }
-
-
 }

Modified: 
hama/trunk/graph/src/main/java/org/apache/hama/graph/GraphJobRunner.java
URL: 
http://svn.apache.org/viewvc/hama/trunk/graph/src/main/java/org/apache/hama/graph/GraphJobRunner.java?rev=1666087&r1=1666086&r2=1666087&view=diff
==============================================================================
--- hama/trunk/graph/src/main/java/org/apache/hama/graph/GraphJobRunner.java 
(original)
+++ hama/trunk/graph/src/main/java/org/apache/hama/graph/GraphJobRunner.java 
Thu Mar 12 07:40:36 2015
@@ -17,6 +17,8 @@
  */
 package org.apache.hama.graph;
 
+import java.io.ByteArrayInputStream;
+import java.io.DataInputStream;
 import java.io.IOException;
 import java.util.Collections;
 import java.util.HashSet;
@@ -229,18 +231,23 @@ public final class GraphJobRunner<V exte
     notComputedVertices = new HashSet();
     notComputedVertices.addAll(vertices.keySet());
 
+    Iterable<Writable> msgs = null;
     Vertex<V, E, M> vertex = null;
 
     while (currentMessage != null) {
       vertex = vertices.get((V) currentMessage.getVertexId());
 
+      final int numOfValues = currentMessage.getNumOfValues();
+      final byte[] serializedMsgs = currentMessage.getValuesBytes();
+      msgs = getIterableMessages(numOfValues, serializedMsgs);
+
       // reactivation
       if (vertex.isHalted()) {
         vertex.setActive();
       }
 
       if (!vertex.isHalted()) {
-        vertex.compute((Iterable<M>) currentMessage.getIterableMessages());
+        vertex.compute((Iterable<M>) msgs);
         vertices.finishVertexComputation(vertex);
         activeVertices++;
 
@@ -643,6 +650,42 @@ public final class GraphJobRunner<V exte
     return (X) ReflectionUtils.newInstance(EDGE_VALUE_CLASS);
   }
 
+  public static Iterable<Writable> getIterableMessages(final int numOfValues,
+      final byte[] msgBytes) {
+
+    return new Iterable<Writable>() {
+      @Override
+      public Iterator<Writable> iterator() {
+        return new Iterator<Writable>() {
+          ByteArrayInputStream bis = new ByteArrayInputStream(msgBytes);
+          DataInputStream dis = new DataInputStream(bis);
+          int index = 0;
+
+          @Override
+          public boolean hasNext() {
+            return (index < numOfValues) ? true : false;
+          }
+
+          @Override
+          public Writable next() {
+            Writable v = GraphJobRunner.createVertexValue();
+            try {
+              v.readFields(dis);
+            } catch (IOException e) {
+              e.printStackTrace();
+            }
+            index++;
+            return v;
+          }
+
+          @Override
+          public void remove() {
+          }
+        };
+      }
+    };
+  }
+
   public int getChangedVertexCnt() {
     return changedVertexCnt;
   }

Modified: 
hama/trunk/graph/src/main/java/org/apache/hama/graph/OutgoingVertexMessageManager.java
URL: 
http://svn.apache.org/viewvc/hama/trunk/graph/src/main/java/org/apache/hama/graph/OutgoingVertexMessageManager.java?rev=1666087&r1=1666086&r2=1666087&view=diff
==============================================================================
--- 
hama/trunk/graph/src/main/java/org/apache/hama/graph/OutgoingVertexMessageManager.java
 (original)
+++ 
hama/trunk/graph/src/main/java/org/apache/hama/graph/OutgoingVertexMessageManager.java
 Thu Mar 12 07:40:36 2015
@@ -39,6 +39,7 @@ public class OutgoingVertexMessageManage
       .getLog(OutgoingVertexMessageManager.class);
 
   private Combiner<Writable> combiner;
+  private Iterable<Writable> msgs;
   private HashMap<InetSocketAddress, MessagePerVertex> storage = new 
HashMap<InetSocketAddress, MessagePerVertex>();
 
   @SuppressWarnings("unchecked")
@@ -49,8 +50,7 @@ public class OutgoingVertexMessageManage
     final String combinerName = conf.get(Constants.COMBINER_CLASS);
     if (combinerName != null) {
       try {
-        combiner = (Combiner<Writable>) ReflectionUtils
-            .newInstance(combinerName);
+        combiner = (Combiner<Writable>) 
ReflectionUtils.newInstance(combinerName);
       } catch (ClassNotFoundException e) {
         e.printStackTrace();
       }
@@ -73,11 +73,13 @@ public class OutgoingVertexMessageManage
       // Combining messages
       if (combiner != null && msgPerVertex.get(vertexID).getNumOfValues() > 1) 
{
 
+        final int numOfValues = msgPerVertex.get(vertexID).getNumOfValues();
+        final byte[] msgBytes = msgPerVertex.get(vertexID).getValuesBytes();
+        msgs = GraphJobRunner.getIterableMessages(numOfValues, msgBytes);
+
         // Overwrite
-        storage.get(targetPeerAddress).put(
-            vertexID,
-            new GraphJobMessage(vertexID, combiner.combine(msgPerVertex.get(
-                vertexID).getIterableMessages())));
+        storage.get(targetPeerAddress).put(vertexID,
+            new GraphJobMessage(vertexID, combiner.combine(msgs)));
       }
     } else {
       outgoingBundles.get(targetPeerAddress).addMessage(msg);

Modified: 
hama/trunk/ml/src/test/java/org/apache/hama/ml/recommendation/TestOnlineCF.java
URL: 
http://svn.apache.org/viewvc/hama/trunk/ml/src/test/java/org/apache/hama/ml/recommendation/TestOnlineCF.java?rev=1666087&r1=1666086&r2=1666087&view=diff
==============================================================================
--- 
hama/trunk/ml/src/test/java/org/apache/hama/ml/recommendation/TestOnlineCF.java 
(original)
+++ 
hama/trunk/ml/src/test/java/org/apache/hama/ml/recommendation/TestOnlineCF.java 
Thu Mar 12 07:40:36 2015
@@ -31,38 +31,41 @@ import org.apache.hama.ml.recommendation
 import org.apache.hama.ml.recommendation.cf.function.MeanAbsError;
 import org.junit.Test;
 
-public class TestOnlineCF extends TestCase {
+public class TestOnlineCF extends TestCase{
   @SuppressWarnings({ "deprecation", "rawtypes", "unchecked" })
   @Test
   public void testOnlineCF() {
-    Preference[] train_prefs = { new Preference<Integer, Integer>(1, 1, 4),
-        new Preference<Integer, Integer>(1, 2, 2.5),
-        new Preference<Integer, Integer>(1, 3, 3.5),
-        new Preference<Integer, Integer>(1, 4, 1),
-        new Preference<Integer, Integer>(1, 5, 3.5),
-        new Preference<Integer, Integer>(2, 1, 4),
-        new Preference<Integer, Integer>(2, 2, 2.5),
-        new Preference<Integer, Integer>(2, 3, 3.5),
-        new Preference<Integer, Integer>(2, 4, 1),
-        new Preference<Integer, Integer>(2, 5, 3.5),
-        new Preference<Integer, Integer>(3, 1, 4),
-        new Preference<Integer, Integer>(3, 2, 2.5),
-        new Preference<Integer, Integer>(3, 3, 3.5) };
-    Preference[] test_prefs = { new Preference<Integer, Integer>(1, 3, 3.5),
-        new Preference<Integer, Integer>(2, 4, 1),
-        new Preference<Integer, Integer>(3, 4, 1),
-        new Preference<Integer, Integer>(3, 5, 3.5) };
-
+    Preference[] train_prefs = {
+                        new Preference<Integer, Integer>(1, 1, 4),
+                        new Preference<Integer, Integer>(1, 2, 2.5),
+                        new Preference<Integer, Integer>(1, 3, 3.5),
+                        new Preference<Integer, Integer>(1, 4, 1),
+                        new Preference<Integer, Integer>(1, 5, 3.5),
+                        new Preference<Integer, Integer>(2, 1, 4),
+                        new Preference<Integer, Integer>(2, 2, 2.5),
+                        new Preference<Integer, Integer>(2, 3, 3.5),
+                        new Preference<Integer, Integer>(2, 4, 1),
+                        new Preference<Integer, Integer>(2, 5, 3.5),
+                        new Preference<Integer, Integer>(3, 1, 4),
+                        new Preference<Integer, Integer>(3, 2, 2.5),
+                        new Preference<Integer, Integer>(3, 3, 3.5)};
+    Preference[] test_prefs = {
+                        new Preference<Integer, Integer>(1, 3, 3.5),
+                        new Preference<Integer, Integer>(2, 4, 1),
+                        new Preference<Integer, Integer>(3, 4, 1),
+                        new Preference<Integer, Integer>(3, 5, 3.5)
+                      };
+    
     Random rnd = new Random();
     Long num = Long.valueOf(rnd.nextInt(100000));
     String fileName = "onlinecf_train" + num.toString();
     String outputFileName = "onlinecf_model" + num.toString();
-
+    
     Configuration fsConf = new Configuration();
     String strDataPath = "/tmp/" + fileName;
     String convertedFileName = "/tmp/converted_" + fileName;
     Path dataPath = new Path(strDataPath);
-
+    
     try {
       URI uri = new URI(strDataPath);
       FileSystem fs = FileSystem.get(uri, fsConf);
@@ -80,11 +83,10 @@ public class TestOnlineCF extends TestCa
       }
       fileOut.writeBytes(str.toString());
       fileOut.close();
-
+      
       MovieLensConverter converter = new MovieLensConverter();
-      assertEquals(true,
-          converter.convert(strDataPath, null, convertedFileName));
-
+      assertEquals(true, converter.convert(strDataPath, null, 
convertedFileName));
+      
       OnlineCF recommender = new OnlineCF();
       recommender.setInputPreferences(convertedFileName);
       recommender.setIteration(150);
@@ -99,12 +101,11 @@ public class TestOnlineCF extends TestCa
       int correct = 0;
       for (Preference<Integer, Integer> test : test_prefs) {
         double actual = test.getValue().get();
-        double estimated = recommender.estimatePreference(test.getUserId(),
-            test.getItemId());
-        correct += (Math.abs(actual - estimated) < 0.5) ? 1 : 0;
+        double estimated = recommender.estimatePreference(test.getUserId(), 
test.getItemId()); 
+        correct += (Math.abs(actual-estimated)<0.5)?1:0;
       }
 
-      assertEquals(test_prefs.length * 0.75, correct, 1);
+      assertEquals(test_prefs.length*0.75, correct, 1);
 
       fs.delete(new Path(outputFileName));
       fs.delete(new Path(strDataPath));

Modified: hama/trunk/pom.xml
URL: 
http://svn.apache.org/viewvc/hama/trunk/pom.xml?rev=1666087&r1=1666086&r2=1666087&view=diff
==============================================================================
--- hama/trunk/pom.xml (original)
+++ hama/trunk/pom.xml Thu Mar 12 07:40:36 2015
@@ -102,7 +102,7 @@
     <log4j.version>1.2.16</log4j.version>
     <zookeeper.version>3.4.5</zookeeper.version>
     <ant.version>1.7.1</ant.version>
-    <kryo.version>3.0.0</kryo.version>
+    <kryo.version>2.20</kryo.version>
   </properties>
 
   <repositories>
@@ -224,7 +224,7 @@
         <version>1.0.5</version>
       </dependency>
        <dependency>
-         <groupId>com.esotericsoftware</groupId>
+         <groupId>com.esotericsoftware.kryo</groupId>
          <artifactId>kryo</artifactId>
          <version>${kryo.version}</version>
       </dependency>


Reply via email to