You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@giraph.apache.org by ni...@apache.org on 2013/01/04 21:52:39 UTC

[1/23] GIRAPH-409: Refactor / cleanups (nitay)

http://git-wip-us.apache.org/repos/asf/giraph/blob/1684891e/giraph-core/src/test/java/org/apache/giraph/vertex/TestVertexTypes.java
----------------------------------------------------------------------
diff --git a/giraph-core/src/test/java/org/apache/giraph/vertex/TestVertexTypes.java b/giraph-core/src/test/java/org/apache/giraph/vertex/TestVertexTypes.java
new file mode 100644
index 0000000..1a260d5
--- /dev/null
+++ b/giraph-core/src/test/java/org/apache/giraph/vertex/TestVertexTypes.java
@@ -0,0 +1,209 @@
+/*
+ * 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.giraph.vertex;
+
+import org.apache.giraph.conf.GiraphConstants;
+import org.apache.giraph.examples.SimpleSuperstepVertex.SimpleSuperstepVertexInputFormat;
+import org.apache.giraph.combiner.Combiner;
+import org.apache.giraph.graph.GiraphTypeValidator;
+import org.apache.giraph.io.VertexInputFormat;
+import org.apache.giraph.io.VertexOutputFormat;
+import org.apache.giraph.io.formats.GeneratedVertexInputFormat;
+import org.apache.giraph.io.formats.JsonBase64VertexInputFormat;
+import org.apache.giraph.io.formats.JsonBase64VertexOutputFormat;
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.io.DoubleWritable;
+import org.apache.hadoop.io.FloatWritable;
+import org.apache.hadoop.io.IntWritable;
+import org.apache.hadoop.io.LongWritable;
+import org.junit.Test;
+
+import java.io.IOException;
+
+
+public class TestVertexTypes {
+
+    /**
+     * Matches the {@link GeneratedVertexInputFormat}
+     */
+    private static class GeneratedVertexMatch extends
+            EdgeListVertex<LongWritable, IntWritable, FloatWritable,
+            FloatWritable> {
+        @Override
+        public void compute(Iterable<FloatWritable> messages)
+            throws IOException {
+        }
+    }
+
+    /**
+     * Matches the {@link GeneratedVertexInputFormat}
+     */
+    private static class DerivedVertexMatch extends GeneratedVertexMatch {
+    }
+
+    /**
+     * Mismatches the {@link GeneratedVertexInputFormat}
+     */
+    private static class GeneratedVertexMismatch extends
+            EdgeListVertex<LongWritable, FloatWritable, FloatWritable,
+            FloatWritable> {
+        @Override
+        public void compute(Iterable<FloatWritable> messages)
+                throws IOException {
+        }
+    }
+
+    /**
+     * Matches the {@link GeneratedVertexMatch}
+     */
+    private static class GeneratedVertexMatchCombiner extends
+        Combiner<LongWritable, FloatWritable> {
+      @Override
+      public void combine(LongWritable vertexIndex,
+          FloatWritable originalMessage,
+          FloatWritable messageToCombine) {
+      }
+
+      @Override
+      public FloatWritable createInitialMessage() {
+        return null;
+      }
+    }
+
+    /**
+     * Mismatches the {@link GeneratedVertexMatch}
+     */
+    private static class GeneratedVertexMismatchCombiner extends
+        Combiner<LongWritable, DoubleWritable> {
+      @Override
+      public void combine(LongWritable vertexIndex,
+          DoubleWritable originalMessage,
+          DoubleWritable messageToCombine) {
+      }
+
+      @Override
+      public DoubleWritable createInitialMessage() {
+        return null;
+      }
+    }
+
+    @Test
+    public void testMatchingType() throws SecurityException,
+            NoSuchMethodException, NoSuchFieldException {
+        Configuration conf = new Configuration();
+        conf.setClass(GiraphConstants.VERTEX_CLASS,
+                      GeneratedVertexMatch.class,
+                      Vertex.class);
+        conf.setClass(GiraphConstants.VERTEX_INPUT_FORMAT_CLASS,
+                      SimpleSuperstepVertexInputFormat.class,
+                      VertexInputFormat.class);
+        conf.setClass(GiraphConstants.VERTEX_COMBINER_CLASS,
+                      GeneratedVertexMatchCombiner.class,
+                      Combiner.class);
+      @SuppressWarnings("rawtypes")
+      GiraphTypeValidator<?, ?, ?, ?> validator =
+        new GiraphTypeValidator(conf);
+      validator.validateClassTypes();
+    }
+
+    @Test
+    public void testDerivedMatchingType() throws SecurityException,
+            NoSuchMethodException, NoSuchFieldException {
+        Configuration conf = new Configuration();
+        conf.setClass(GiraphConstants.VERTEX_CLASS,
+                      DerivedVertexMatch.class,
+                      Vertex.class);
+        conf.setClass(GiraphConstants.VERTEX_INPUT_FORMAT_CLASS,
+                      SimpleSuperstepVertexInputFormat.class,
+                      VertexInputFormat.class);
+        @SuppressWarnings("rawtypes")
+        GiraphTypeValidator<?, ?, ?, ?> validator =
+          new GiraphTypeValidator(conf);
+        validator.validateClassTypes();
+    }
+
+    @Test
+    public void testDerivedInputFormatType() throws SecurityException,
+            NoSuchMethodException, NoSuchFieldException {
+        Configuration conf = new Configuration();
+        conf.setClass(GiraphConstants.VERTEX_CLASS,
+                      DerivedVertexMatch.class,
+                      Vertex.class);
+        conf.setClass(GiraphConstants.VERTEX_INPUT_FORMAT_CLASS,
+                      SimpleSuperstepVertexInputFormat.class,
+                      VertexInputFormat.class);
+      @SuppressWarnings("rawtypes")
+      GiraphTypeValidator<?, ?, ?, ?> validator =
+        new GiraphTypeValidator(conf);
+      validator.validateClassTypes();
+    }
+
+    @Test(expected = IllegalArgumentException.class)
+    public void testMismatchingVertex() throws SecurityException,
+      NoSuchMethodException, NoSuchFieldException {
+      Configuration conf = new Configuration();
+      conf.setClass(GiraphConstants.VERTEX_CLASS,
+        GeneratedVertexMismatch.class,
+        Vertex.class);
+        conf.setClass(GiraphConstants.VERTEX_INPUT_FORMAT_CLASS,
+          SimpleSuperstepVertexInputFormat.class,
+          VertexInputFormat.class);
+        @SuppressWarnings("rawtypes")
+        GiraphTypeValidator<?, ?, ?, ?> validator =
+          new GiraphTypeValidator(conf);
+        validator.validateClassTypes();
+    }
+
+    @Test(expected = IllegalArgumentException.class)
+    public void testMismatchingCombiner() throws SecurityException,
+      NoSuchMethodException, NoSuchFieldException {
+      Configuration conf = new Configuration();
+      conf.setClass(GiraphConstants.VERTEX_CLASS,
+        GeneratedVertexMatch.class, Vertex.class);
+      conf.setClass(GiraphConstants.VERTEX_INPUT_FORMAT_CLASS,
+        SimpleSuperstepVertexInputFormat.class,
+        VertexInputFormat.class);
+      conf.setClass(GiraphConstants.VERTEX_COMBINER_CLASS,
+        GeneratedVertexMismatchCombiner.class,
+        Combiner.class);
+      @SuppressWarnings("rawtypes")
+      GiraphTypeValidator<?, ?, ?, ?> validator =
+        new GiraphTypeValidator(conf);
+      validator.validateClassTypes();
+    }
+
+    @Test
+    public void testJsonBase64FormatType() throws SecurityException,
+            NoSuchMethodException, NoSuchFieldException {
+        Configuration conf = new Configuration();
+        conf.setClass(GiraphConstants.VERTEX_CLASS,
+                      GeneratedVertexMatch.class,
+                      Vertex.class);
+        conf.setClass(GiraphConstants.VERTEX_INPUT_FORMAT_CLASS,
+                      JsonBase64VertexInputFormat.class,
+                      VertexInputFormat.class);
+        conf.setClass(GiraphConstants.VERTEX_OUTPUT_FORMAT_CLASS,
+                      JsonBase64VertexOutputFormat.class,
+                      VertexOutputFormat.class);
+        @SuppressWarnings("rawtypes")
+        GiraphTypeValidator<?, ?, ?, ?> validator =
+          new GiraphTypeValidator(conf);
+        validator.validateClassTypes();
+    }
+}

http://git-wip-us.apache.org/repos/asf/giraph/blob/1684891e/giraph-core/src/test/java/org/apache/giraph/zk/TestPredicateLock.java
----------------------------------------------------------------------
diff --git a/giraph-core/src/test/java/org/apache/giraph/zk/TestPredicateLock.java b/giraph-core/src/test/java/org/apache/giraph/zk/TestPredicateLock.java
new file mode 100644
index 0000000..9f32570
--- /dev/null
+++ b/giraph-core/src/test/java/org/apache/giraph/zk/TestPredicateLock.java
@@ -0,0 +1,148 @@
+/*
+ * 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.giraph.zk;
+
+import static org.junit.Assert.assertEquals;
+import static org.junit.Assert.assertFalse;
+import static org.junit.Assert.assertTrue;
+import static org.mockito.Mockito.mock;
+import static org.mockito.Mockito.when;
+
+import java.util.concurrent.atomic.AtomicInteger;
+
+import org.apache.giraph.time.Time;
+import org.apache.giraph.zk.BspEvent;
+import org.apache.giraph.zk.PredicateLock;
+import org.apache.hadoop.util.Progressable;
+import org.junit.Before;
+import org.junit.Test;
+
+/**
+ * Ensure that PredicateLock objects work correctly.
+ */
+public class TestPredicateLock {
+  /** How many times was progress called? */
+  private AtomicInteger progressCalled = new AtomicInteger(0);
+
+  private static class SignalThread extends Thread {
+    private final BspEvent event;
+    public SignalThread(BspEvent event) {
+      this.event = event;
+    }
+    public void run() {
+      try {
+        Thread.sleep(500);
+      } catch (InterruptedException e) {
+      }
+      event.signal();
+    }
+  }
+
+  private Progressable stubContext;
+
+  private Progressable getStubProgressable() {
+    if (stubContext == null)
+      stubContext = new Progressable() {
+        @Override
+        public void progress() {
+          progressCalled.incrementAndGet();
+        }
+      };
+    return stubContext;
+  }
+
+  @Before
+  public void setUp() {
+    progressCalled.set(0);
+  }
+
+  /**
+   * SMake sure the the event is not signaled.
+   */
+  @Test
+  public void testWaitMsecsNoEvent() {
+    Time mockTime = mock(Time.class);
+    when(mockTime.getMilliseconds()).
+        thenReturn(0L).thenReturn(2L);
+    BspEvent event = new PredicateLock(getStubProgressable(), 1, mockTime);
+    boolean gotPredicate = event.waitMsecs(1);
+    assertFalse(gotPredicate);
+    assertEquals(0, progressCalled.get());
+    when(mockTime.getMilliseconds()).
+        thenReturn(0L).thenReturn(0L).thenReturn(2L);
+    gotPredicate = event.waitMsecs(1);
+    assertFalse(gotPredicate);
+    assertEquals(1, progressCalled.get());
+  }
+
+  /**
+   * Single threaded case where the event is signaled.
+   */
+  @Test
+  public void testEvent() {
+    Time mockTime = mock(Time.class);
+    when(mockTime.getMilliseconds()).
+        thenReturn(0L).thenReturn(2L);
+    BspEvent event = new PredicateLock(getStubProgressable(), 1, mockTime);
+    event.signal();
+    boolean gotPredicate = event.waitMsecs(2);
+    assertTrue(gotPredicate);
+    event.reset();
+    when(mockTime.getMilliseconds()).
+        thenReturn(0L).thenReturn(2L);
+    gotPredicate = event.waitMsecs(0);
+    assertFalse(gotPredicate);
+  }
+
+  /**
+   * Thread signaled test for {@link PredicateLock#waitForever()}
+   */
+  @Test
+  public void testWaitForever() {
+    BspEvent event = new PredicateLock(getStubProgressable());
+    Thread signalThread = new SignalThread(event);
+    signalThread.start();
+    event.waitForever();
+    try {
+      signalThread.join();
+    } catch (InterruptedException e) {
+    }
+    assertTrue(event.waitMsecs(0));
+  }
+
+  /**
+   * Thread signaled test to make sure the the event is signaled correctly
+   *
+   * @throws InterruptedException
+   */
+  @Test
+  public void testWaitMsecs() {
+    BspEvent event = new PredicateLock(getStubProgressable());
+    Thread signalThread = new SignalThread(event);
+    signalThread.start();
+    boolean gotPredicate = event.waitMsecs(2000);
+    assertTrue(gotPredicate);
+    try {
+      signalThread.join();
+    } catch (InterruptedException e) {
+    }
+    gotPredicate = event.waitMsecs(0);
+    assertTrue(gotPredicate);
+  }
+}

http://git-wip-us.apache.org/repos/asf/giraph/blob/1684891e/giraph-core/src/test/java/org/apache/giraph/zk/TestZooKeeperExt.java
----------------------------------------------------------------------
diff --git a/giraph-core/src/test/java/org/apache/giraph/zk/TestZooKeeperExt.java b/giraph-core/src/test/java/org/apache/giraph/zk/TestZooKeeperExt.java
new file mode 100644
index 0000000..3823c58
--- /dev/null
+++ b/giraph-core/src/test/java/org/apache/giraph/zk/TestZooKeeperExt.java
@@ -0,0 +1,173 @@
+/*
+ * 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.giraph.zk;
+
+import static org.junit.Assert.assertEquals;
+import static org.junit.Assert.assertTrue;
+
+import java.util.List;
+
+import org.apache.giraph.zk.ZooKeeperExt;
+import org.apache.zookeeper.CreateMode;
+import org.apache.zookeeper.KeeperException;
+import org.apache.zookeeper.WatchedEvent;
+import org.apache.zookeeper.Watcher;
+import org.apache.zookeeper.ZooDefs.Ids;
+import org.junit.After;
+import org.junit.Before;
+import org.junit.Test;
+
+/**
+ * Test the ZooKeeperExt class.
+ */
+public class TestZooKeeperExt implements Watcher {
+  /** ZooKeeperExt instance */
+  private ZooKeeperExt zooKeeperExt = null;
+  /** ZooKeeper server list */
+  private String zkList = System.getProperty("prop.zookeeper.list");
+
+  public static final String BASE_PATH = "/_zooKeeperExtTest";
+  public static final String FIRST_PATH = "/_first";
+
+  public void process(WatchedEvent event) {
+    return;
+  }
+
+  @Before
+  public void setUp() {
+    try {
+      if (zkList == null) {
+        return;
+      }
+      zooKeeperExt =
+          new ZooKeeperExt(zkList, 30 * 1000, 0, 0, this);
+      zooKeeperExt.deleteExt(BASE_PATH, -1, true);
+    } catch (KeeperException.NoNodeException e) {
+      System.out.println("Clean start: No node " + BASE_PATH);
+    } catch (Exception e) {
+      throw new RuntimeException(e);
+    }
+  }
+
+  @After
+  public void tearDown() {
+    if (zooKeeperExt == null) {
+      return;
+    }
+    try {
+      zooKeeperExt.close();
+    } catch (InterruptedException e) {
+      throw new RuntimeException(e);
+    }
+  }
+
+  @Test
+  public void testCreateExt() throws KeeperException, InterruptedException {
+    if (zooKeeperExt == null) {
+      System.out.println(
+          "testCreateExt: No prop.zookeeper.list set, skipping test");
+      return;
+    }
+    System.out.println("Created: " +
+                           zooKeeperExt.createExt(
+                               BASE_PATH + FIRST_PATH,
+                               null,
+                               Ids.OPEN_ACL_UNSAFE,
+                               CreateMode.PERSISTENT,
+                               true));
+    zooKeeperExt.deleteExt(BASE_PATH + FIRST_PATH, -1, false);
+    zooKeeperExt.deleteExt(BASE_PATH, -1, false);
+  }
+
+  @Test
+  public void testDeleteExt() throws KeeperException, InterruptedException {
+    if (zooKeeperExt == null) {
+      System.out.println(
+          "testDeleteExt: No prop.zookeeper.list set, skipping test");
+      return;
+    }
+    zooKeeperExt.createExt(BASE_PATH,
+                           null,
+                           Ids.OPEN_ACL_UNSAFE,
+                           CreateMode.PERSISTENT,
+                           false);
+    zooKeeperExt.createExt(BASE_PATH + FIRST_PATH,
+                           null,
+                           Ids.OPEN_ACL_UNSAFE,
+                           CreateMode.PERSISTENT,
+                           false);
+    try {
+      zooKeeperExt.deleteExt(BASE_PATH, -1, false);
+    } catch (KeeperException.NotEmptyException e) {
+      System.out.println(
+          "Correctly failed to delete since not recursive");
+    }
+    zooKeeperExt.deleteExt(BASE_PATH, -1, true);
+  }
+
+  @Test
+  public void testGetChildrenExt()
+      throws KeeperException, InterruptedException {
+    if (zooKeeperExt == null) {
+      System.out.println(
+          "testGetChildrenExt: No prop.zookeeper.list set, skipping test");
+      return;
+    }
+    zooKeeperExt.createExt(BASE_PATH,
+                           null,
+                           Ids.OPEN_ACL_UNSAFE,
+                           CreateMode.PERSISTENT,
+                           false);
+    zooKeeperExt.createExt(BASE_PATH + "/b",
+                           null,
+                           Ids.OPEN_ACL_UNSAFE,
+                           CreateMode.PERSISTENT_SEQUENTIAL,
+                           false);
+    zooKeeperExt.createExt(BASE_PATH + "/a",
+                           null,
+                           Ids.OPEN_ACL_UNSAFE,
+                           CreateMode.PERSISTENT_SEQUENTIAL,
+                           false);
+    zooKeeperExt.createExt(BASE_PATH + "/d",
+                           null,
+                           Ids.OPEN_ACL_UNSAFE,
+                           CreateMode.PERSISTENT_SEQUENTIAL,
+                           false);
+    zooKeeperExt.createExt(BASE_PATH + "/c",
+                           null,
+                           Ids.OPEN_ACL_UNSAFE,
+                           CreateMode.PERSISTENT_SEQUENTIAL,
+                           false);
+    List<String> fullPathList =
+        zooKeeperExt.getChildrenExt(BASE_PATH, false, false, true);
+    for (String fullPath : fullPathList) {
+      assertTrue(fullPath.contains(BASE_PATH + "/"));
+    }
+    List<String> sequenceOrderedList =
+        zooKeeperExt.getChildrenExt(BASE_PATH, false, true, true);
+    for (String fullPath : sequenceOrderedList) {
+      assertTrue(fullPath.contains(BASE_PATH + "/"));
+    }
+    assertEquals(4, sequenceOrderedList.size());
+    assertTrue(sequenceOrderedList.get(0).contains("/b"));
+    assertTrue(sequenceOrderedList.get(1).contains("/a"));
+    assertTrue(sequenceOrderedList.get(2).contains("/d"));
+    assertTrue(sequenceOrderedList.get(3).contains("/c"));
+  }
+}

http://git-wip-us.apache.org/repos/asf/giraph/blob/1684891e/giraph-core/src/test/java/org/apache/giraph/zk/TestZooKeeperManager.java
----------------------------------------------------------------------
diff --git a/giraph-core/src/test/java/org/apache/giraph/zk/TestZooKeeperManager.java b/giraph-core/src/test/java/org/apache/giraph/zk/TestZooKeeperManager.java
index 50d4420..c335be5 100644
--- a/giraph-core/src/test/java/org/apache/giraph/zk/TestZooKeeperManager.java
+++ b/giraph-core/src/test/java/org/apache/giraph/zk/TestZooKeeperManager.java
@@ -18,7 +18,6 @@
 package org.apache.giraph.zk;
 
 import org.apache.giraph.conf.GiraphConstants;
-import org.apache.giraph.zk.ZooKeeperManager;
 import org.apache.hadoop.conf.Configuration;
 import org.junit.Test;
 

http://git-wip-us.apache.org/repos/asf/giraph/blob/1684891e/giraph-hbase/src/main/java/org/apache/giraph/io/hbase/HBaseVertexInputFormat.java
----------------------------------------------------------------------
diff --git a/giraph-hbase/src/main/java/org/apache/giraph/io/hbase/HBaseVertexInputFormat.java b/giraph-hbase/src/main/java/org/apache/giraph/io/hbase/HBaseVertexInputFormat.java
index cf87035..c16489a 100644
--- a/giraph-hbase/src/main/java/org/apache/giraph/io/hbase/HBaseVertexInputFormat.java
+++ b/giraph-hbase/src/main/java/org/apache/giraph/io/hbase/HBaseVertexInputFormat.java
@@ -18,8 +18,8 @@
 package org.apache.giraph.io.hbase;
 
 import org.apache.giraph.conf.ImmutableClassesGiraphConfiguration;
-import org.apache.giraph.graph.VertexInputFormat;
-import org.apache.giraph.graph.VertexReader;
+import org.apache.giraph.io.VertexInputFormat;
+import org.apache.giraph.io.VertexReader;
 import org.apache.hadoop.hbase.client.Result;
 import org.apache.hadoop.hbase.io.ImmutableBytesWritable;
 import org.apache.hadoop.hbase.mapreduce.TableInputFormat;

http://git-wip-us.apache.org/repos/asf/giraph/blob/1684891e/giraph-hbase/src/main/java/org/apache/giraph/io/hbase/HBaseVertexOutputFormat.java
----------------------------------------------------------------------
diff --git a/giraph-hbase/src/main/java/org/apache/giraph/io/hbase/HBaseVertexOutputFormat.java b/giraph-hbase/src/main/java/org/apache/giraph/io/hbase/HBaseVertexOutputFormat.java
index 2a27b63..2971397 100644
--- a/giraph-hbase/src/main/java/org/apache/giraph/io/hbase/HBaseVertexOutputFormat.java
+++ b/giraph-hbase/src/main/java/org/apache/giraph/io/hbase/HBaseVertexOutputFormat.java
@@ -18,8 +18,8 @@
 
 package org.apache.giraph.io.hbase;
 
-import org.apache.giraph.graph.VertexOutputFormat;
-import org.apache.giraph.graph.VertexWriter;
+import org.apache.giraph.io.VertexOutputFormat;
+import org.apache.giraph.io.VertexWriter;
 import org.apache.hadoop.hbase.io.ImmutableBytesWritable;
 import org.apache.hadoop.hbase.mapreduce.TableOutputFormat;
 import org.apache.hadoop.io.Writable;

http://git-wip-us.apache.org/repos/asf/giraph/blob/1684891e/giraph-hbase/src/test/java/org/apache/giraph/io/hbase/TestHBaseRootMarkerVertextFormat.java
----------------------------------------------------------------------
diff --git a/giraph-hbase/src/test/java/org/apache/giraph/io/hbase/TestHBaseRootMarkerVertextFormat.java b/giraph-hbase/src/test/java/org/apache/giraph/io/hbase/TestHBaseRootMarkerVertextFormat.java
index ea4bed1..77738b5 100644
--- a/giraph-hbase/src/test/java/org/apache/giraph/io/hbase/TestHBaseRootMarkerVertextFormat.java
+++ b/giraph-hbase/src/test/java/org/apache/giraph/io/hbase/TestHBaseRootMarkerVertextFormat.java
@@ -21,7 +21,7 @@ package org.apache.giraph.io.hbase;
 
 import org.apache.giraph.BspCase;
 import org.apache.giraph.conf.GiraphConfiguration;
-import org.apache.giraph.graph.EdgeListVertex;
+import org.apache.giraph.vertex.EdgeListVertex;
 import org.apache.giraph.graph.GiraphJob;
 import org.apache.giraph.io.hbase.edgemarker.TableEdgeInputFormat;
 import org.apache.giraph.io.hbase.edgemarker.TableEdgeOutputFormat;

http://git-wip-us.apache.org/repos/asf/giraph/blob/1684891e/giraph-hbase/src/test/java/org/apache/giraph/io/hbase/edgemarker/TableEdgeInputFormat.java
----------------------------------------------------------------------
diff --git a/giraph-hbase/src/test/java/org/apache/giraph/io/hbase/edgemarker/TableEdgeInputFormat.java b/giraph-hbase/src/test/java/org/apache/giraph/io/hbase/edgemarker/TableEdgeInputFormat.java
index e4e08d6..18fee52 100644
--- a/giraph-hbase/src/test/java/org/apache/giraph/io/hbase/edgemarker/TableEdgeInputFormat.java
+++ b/giraph-hbase/src/test/java/org/apache/giraph/io/hbase/edgemarker/TableEdgeInputFormat.java
@@ -18,8 +18,8 @@
 package org.apache.giraph.io.hbase.edgemarker;
 
 import org.apache.giraph.graph.Edge;
-import org.apache.giraph.graph.Vertex;
-import org.apache.giraph.graph.VertexReader;
+import org.apache.giraph.vertex.Vertex;
+import org.apache.giraph.io.VertexReader;
 import org.apache.giraph.io.hbase.HBaseVertexInputFormat;
 import org.apache.hadoop.hbase.client.Result;
 import org.apache.hadoop.hbase.util.Bytes;

http://git-wip-us.apache.org/repos/asf/giraph/blob/1684891e/giraph-hbase/src/test/java/org/apache/giraph/io/hbase/edgemarker/TableEdgeOutputFormat.java
----------------------------------------------------------------------
diff --git a/giraph-hbase/src/test/java/org/apache/giraph/io/hbase/edgemarker/TableEdgeOutputFormat.java b/giraph-hbase/src/test/java/org/apache/giraph/io/hbase/edgemarker/TableEdgeOutputFormat.java
index 169fd88..a131357 100644
--- a/giraph-hbase/src/test/java/org/apache/giraph/io/hbase/edgemarker/TableEdgeOutputFormat.java
+++ b/giraph-hbase/src/test/java/org/apache/giraph/io/hbase/edgemarker/TableEdgeOutputFormat.java
@@ -18,8 +18,8 @@
 package org.apache.giraph.io.hbase.edgemarker;
 
 import org.apache.giraph.io.hbase.HBaseVertexOutputFormat;
-import org.apache.giraph.graph.Vertex;
-import org.apache.giraph.graph.VertexWriter;
+import org.apache.giraph.vertex.Vertex;
+import org.apache.giraph.io.VertexWriter;
 import org.apache.hadoop.hbase.client.Put;
 import org.apache.hadoop.hbase.io.ImmutableBytesWritable;
 import org.apache.hadoop.hbase.util.Bytes;

http://git-wip-us.apache.org/repos/asf/giraph/blob/1684891e/giraph-hcatalog/src/main/java/org/apache/giraph/io/hcatalog/HCatalogEdgeInputFormat.java
----------------------------------------------------------------------
diff --git a/giraph-hcatalog/src/main/java/org/apache/giraph/io/hcatalog/HCatalogEdgeInputFormat.java b/giraph-hcatalog/src/main/java/org/apache/giraph/io/hcatalog/HCatalogEdgeInputFormat.java
index 2112df3..018972e 100644
--- a/giraph-hcatalog/src/main/java/org/apache/giraph/io/hcatalog/HCatalogEdgeInputFormat.java
+++ b/giraph-hcatalog/src/main/java/org/apache/giraph/io/hcatalog/HCatalogEdgeInputFormat.java
@@ -19,8 +19,8 @@
 package org.apache.giraph.io.hcatalog;
 
 import org.apache.giraph.graph.Edge;
-import org.apache.giraph.graph.EdgeInputFormat;
-import org.apache.giraph.graph.EdgeReader;
+import org.apache.giraph.io.EdgeInputFormat;
+import org.apache.giraph.io.EdgeReader;
 import org.apache.giraph.graph.EdgeWithSource;
 import org.apache.hadoop.io.Writable;
 import org.apache.hadoop.io.WritableComparable;

http://git-wip-us.apache.org/repos/asf/giraph/blob/1684891e/giraph-hcatalog/src/main/java/org/apache/giraph/io/hcatalog/HCatalogVertexInputFormat.java
----------------------------------------------------------------------
diff --git a/giraph-hcatalog/src/main/java/org/apache/giraph/io/hcatalog/HCatalogVertexInputFormat.java b/giraph-hcatalog/src/main/java/org/apache/giraph/io/hcatalog/HCatalogVertexInputFormat.java
index ec49137..52b9ae3 100644
--- a/giraph-hcatalog/src/main/java/org/apache/giraph/io/hcatalog/HCatalogVertexInputFormat.java
+++ b/giraph-hcatalog/src/main/java/org/apache/giraph/io/hcatalog/HCatalogVertexInputFormat.java
@@ -21,9 +21,9 @@ package org.apache.giraph.io.hcatalog;
 import com.google.common.collect.Lists;
 import org.apache.giraph.conf.ImmutableClassesGiraphConfiguration;
 import org.apache.giraph.graph.Edge;
-import org.apache.giraph.graph.Vertex;
-import org.apache.giraph.graph.VertexInputFormat;
-import org.apache.giraph.graph.VertexReader;
+import org.apache.giraph.vertex.Vertex;
+import org.apache.giraph.io.VertexInputFormat;
+import org.apache.giraph.io.VertexReader;
 import org.apache.giraph.utils.TimedLogger;
 import org.apache.hadoop.io.Writable;
 import org.apache.hadoop.io.WritableComparable;

http://git-wip-us.apache.org/repos/asf/giraph/blob/1684891e/giraph-hcatalog/src/main/java/org/apache/giraph/io/hcatalog/HCatalogVertexOutputFormat.java
----------------------------------------------------------------------
diff --git a/giraph-hcatalog/src/main/java/org/apache/giraph/io/hcatalog/HCatalogVertexOutputFormat.java b/giraph-hcatalog/src/main/java/org/apache/giraph/io/hcatalog/HCatalogVertexOutputFormat.java
index 94c7b85..4bab7dd 100644
--- a/giraph-hcatalog/src/main/java/org/apache/giraph/io/hcatalog/HCatalogVertexOutputFormat.java
+++ b/giraph-hcatalog/src/main/java/org/apache/giraph/io/hcatalog/HCatalogVertexOutputFormat.java
@@ -18,9 +18,9 @@
 
 package org.apache.giraph.io.hcatalog;
 
-import org.apache.giraph.graph.Vertex;
-import org.apache.giraph.graph.VertexOutputFormat;
-import org.apache.giraph.graph.VertexWriter;
+import org.apache.giraph.vertex.Vertex;
+import org.apache.giraph.io.VertexOutputFormat;
+import org.apache.giraph.io.VertexWriter;
 import org.apache.hadoop.io.Writable;
 import org.apache.hadoop.io.WritableComparable;
 import org.apache.hadoop.mapreduce.JobContext;

http://git-wip-us.apache.org/repos/asf/giraph/blob/1684891e/giraph-hcatalog/src/main/java/org/apache/giraph/io/hcatalog/HCatalogVertexValueInputFormat.java
----------------------------------------------------------------------
diff --git a/giraph-hcatalog/src/main/java/org/apache/giraph/io/hcatalog/HCatalogVertexValueInputFormat.java b/giraph-hcatalog/src/main/java/org/apache/giraph/io/hcatalog/HCatalogVertexValueInputFormat.java
index d08179d..b3934e4 100644
--- a/giraph-hcatalog/src/main/java/org/apache/giraph/io/hcatalog/HCatalogVertexValueInputFormat.java
+++ b/giraph-hcatalog/src/main/java/org/apache/giraph/io/hcatalog/HCatalogVertexValueInputFormat.java
@@ -18,8 +18,8 @@
 
 package org.apache.giraph.io.hcatalog;
 
-import org.apache.giraph.graph.VertexValueInputFormat;
-import org.apache.giraph.graph.VertexValueReader;
+import org.apache.giraph.io.VertexValueInputFormat;
+import org.apache.giraph.io.VertexValueReader;
 import org.apache.hadoop.io.Writable;
 import org.apache.hadoop.io.WritableComparable;
 import org.apache.hadoop.mapreduce.InputSplit;

http://git-wip-us.apache.org/repos/asf/giraph/blob/1684891e/giraph-hcatalog/src/main/java/org/apache/giraph/io/hcatalog/HiveGiraphRunner.java
----------------------------------------------------------------------
diff --git a/giraph-hcatalog/src/main/java/org/apache/giraph/io/hcatalog/HiveGiraphRunner.java b/giraph-hcatalog/src/main/java/org/apache/giraph/io/hcatalog/HiveGiraphRunner.java
index 7a7c2f8..20d13ec 100644
--- a/giraph-hcatalog/src/main/java/org/apache/giraph/io/hcatalog/HiveGiraphRunner.java
+++ b/giraph-hcatalog/src/main/java/org/apache/giraph/io/hcatalog/HiveGiraphRunner.java
@@ -24,11 +24,11 @@ import org.apache.commons.cli.GnuParser;
 import org.apache.commons.cli.HelpFormatter;
 import org.apache.commons.cli.Options;
 import org.apache.commons.cli.ParseException;
-import org.apache.giraph.graph.EdgeInputFormat;
+import org.apache.giraph.io.EdgeInputFormat;
 import org.apache.giraph.graph.GiraphJob;
-import org.apache.giraph.graph.Vertex;
-import org.apache.giraph.graph.VertexInputFormat;
-import org.apache.giraph.graph.VertexOutputFormat;
+import org.apache.giraph.vertex.Vertex;
+import org.apache.giraph.io.VertexInputFormat;
+import org.apache.giraph.io.VertexOutputFormat;
 import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.hive.conf.HiveConf;
 import org.apache.hadoop.util.Tool;