reintegrate fullstack_dynamic_deployment
diff --git a/pregelix/pregelix-example/data/clique2/clique.txt b/pregelix/pregelix-example/data/clique2/clique.txt
new file mode 100755
index 0000000..68fbaea
--- /dev/null
+++ b/pregelix/pregelix-example/data/clique2/clique.txt
@@ -0,0 +1,6 @@
+1 2 3 4
+2 1 3
+3 1 2 4 5
+4 1 3
+5 3 6
+6 5
\ No newline at end of file
diff --git a/pregelix/pregelix-example/data/clique3/clique.txt b/pregelix/pregelix-example/data/clique3/clique.txt
new file mode 100755
index 0000000..ed0b7c1
--- /dev/null
+++ b/pregelix/pregelix-example/data/clique3/clique.txt
@@ -0,0 +1,20 @@
+0 1 19
+1 2 3 4 5 6 7 8 9
+2 1 3 4 5 6 7 8 9
+3 1 2 4 5 6 7 8 9
+4 1 2 3 5 6 7 8 9
+5 1 2 3 4 6 7 8 9
+6 1 2 3 4 5 7 8 9
+7 1 2 3 4 5 6 8 9
+8 1 2 3 4 5 6 7 9
+9 1 2 3 4 5 6 7 8 10
+10 9 11
+11 10 12 13 14 15 16 17 18 19
+12 11 13 14 15 16 17 18 19
+13 11 12 14 15 16 17 18 19
+14 11 12 13 15 16 17 18 19
+15 11 12 13 14 16 17 18 19
+16 11 12 13 14 15 17 18 19
+17 11 12 13 14 15 16 18 19
+18 11 12 13 14 15 16 17 19
+19 0 11 12 13 14 15 16 17 18
\ No newline at end of file
diff --git a/pregelix/pregelix-example/pom.xml b/pregelix/pregelix-example/pom.xml
index 37da84f..7ca7e5d 100644
--- a/pregelix/pregelix-example/pom.xml
+++ b/pregelix/pregelix-example/pom.xml
@@ -69,7 +69,7 @@
 				<version>2.7.2</version>
 				<configuration>
 					<forkMode>pertest</forkMode>
-					<argLine>-enableassertions -Xmx512m -XX:MaxPermSize=300m -Dfile.encoding=UTF-8
+					<argLine>-enableassertions -Xmx2047m -XX:MaxPermSize=300m -Dfile.encoding=UTF-8
 						-Djava.util.logging.config.file=src/test/resources/logging.properties</argLine>
 					<includes>
 						<include>**/*TestSuite.java</include>
diff --git a/pregelix/pregelix-example/src/main/java/edu/uci/ics/pregelix/example/inputformat/TextPageRankInputFormat.java b/pregelix/pregelix-example/src/main/java/edu/uci/ics/pregelix/example/inputformat/TextPageRankInputFormat.java
index a8a752e..8d0b776 100644
--- a/pregelix/pregelix-example/src/main/java/edu/uci/ics/pregelix/example/inputformat/TextPageRankInputFormat.java
+++ b/pregelix/pregelix-example/src/main/java/edu/uci/ics/pregelix/example/inputformat/TextPageRankInputFormat.java
@@ -39,7 +39,7 @@
     @Override
     public VertexReader<VLongWritable, DoubleWritable, FloatWritable, DoubleWritable> createVertexReader(
             InputSplit split, TaskAttemptContext context) throws IOException {
-        return new TextPageRankGraphReader(textInputFormat.createRecordReader(split, context));
+        return new TextPageRankGraphReader(textInputFormat.createRecordReader(split, context), context);
     }
 }
 
@@ -52,7 +52,7 @@
     private List<VLongWritable> pool = new ArrayList<VLongWritable>();
     private int used = 0;
 
-    public TextPageRankGraphReader(RecordReader<LongWritable, Text> lineRecordReader) {
+    public TextPageRankGraphReader(RecordReader<LongWritable, Text> lineRecordReader, TaskAttemptContext context) {
         super(lineRecordReader);
     }
 
diff --git a/pregelix/pregelix-example/src/main/java/edu/uci/ics/pregelix/example/maximalclique/CliquesWritable.java b/pregelix/pregelix-example/src/main/java/edu/uci/ics/pregelix/example/maximalclique/CliquesWritable.java
index 0e22ea1..d2c5e6f 100644
--- a/pregelix/pregelix-example/src/main/java/edu/uci/ics/pregelix/example/maximalclique/CliquesWritable.java
+++ b/pregelix/pregelix-example/src/main/java/edu/uci/ics/pregelix/example/maximalclique/CliquesWritable.java
@@ -32,6 +32,7 @@
 
     private List<VLongWritable> cliques = new ArrayList<VLongWritable>();
     private int sizeOfClique = 0;
+    private VLongWritable srcId = new VLongWritable(0);
 
     public CliquesWritable(List<VLongWritable> cliques, int sizeOfClique) {
         this.cliques = cliques;
@@ -43,6 +44,16 @@
     }
 
     /**
+     * Set the srcId
+     * 
+     * @param srcId
+     *            the source vertex Id
+     */
+    public void setSrcId(VLongWritable srcId) {
+        this.srcId = srcId;
+    }
+
+    /**
      * Set the size of cliques.
      * 
      * @param sizeOfClique
@@ -103,6 +114,11 @@
                 cliques.add(vid);
             }
         }
+
+        if (srcId == null) {
+            srcId = new VLongWritable();
+        }
+        srcId.readFields(input);
     }
 
     @Override
@@ -117,6 +133,8 @@
         for (int i = 0; i < cliques.size(); i++) {
             cliques.get(i).write(output);
         }
+
+        srcId.write(output);
     }
 
     @Override
@@ -126,11 +144,14 @@
         StringBuffer sb = new StringBuffer();
         int numCliques = cliques.size() / sizeOfClique;
         for (int i = 0; i < numCliques; i++) {
+            sb.append(srcId);
+            sb.append(",");
+            int start = i * sizeOfClique;
             for (int j = 0; j < sizeOfClique - 1; j++) {
-                sb.append(cliques.get(j));
+                sb.append(cliques.get(start + j));
                 sb.append(",");
             }
-            sb.append(cliques.get(sizeOfClique - 1));
+            sb.append(cliques.get(start + sizeOfClique - 1));
             sb.append(";");
         }
         return sb.toString();
diff --git a/pregelix/pregelix-example/src/main/java/edu/uci/ics/pregelix/example/maximalclique/MaximalCliqueVertex.java b/pregelix/pregelix-example/src/main/java/edu/uci/ics/pregelix/example/maximalclique/MaximalCliqueVertex.java
index 266feb7..85a139e 100644
--- a/pregelix/pregelix-example/src/main/java/edu/uci/ics/pregelix/example/maximalclique/MaximalCliqueVertex.java
+++ b/pregelix/pregelix-example/src/main/java/edu/uci/ics/pregelix/example/maximalclique/MaximalCliqueVertex.java
@@ -44,8 +44,8 @@
 
 /**
  * The maximal clique example -- find maximal cliques in an undirected graph.
- * The result cliques contains vertexes ordered by the vertex id ascendingly. The algorithm takes
- * advantage of that property to do effective pruning.
+ * The result cliques contains vertexes ordered by the vertex id ascendingly.
+ * The algorithm takes advantage of that property to do effective pruning.
  */
 public class MaximalCliqueVertex extends Vertex<VLongWritable, CliquesWritable, NullWritable, AdjacencyListWritable> {
 
@@ -55,7 +55,7 @@
     private int largestCliqueSizeSoFar = 0;
     private List<BitSet> currentMaximalCliques = new ArrayList<BitSet>();
     private CliquesWritable tmpValue = new CliquesWritable();
-    private List<VLongWritable> cliques = new ArrayList<VLongWritable>();
+    private List<VLongWritable> cliqueList = new ArrayList<VLongWritable>();
 
     /**
      * Update the current maximal cliques
@@ -68,7 +68,6 @@
         vertexList.clear();
         invertedMap.clear();
         currentMaximalCliques.clear();
-        cliques.clear();
         tmpValue.reset();
 
         // build the initial sub graph
@@ -76,8 +75,6 @@
             AdjacencyListWritable adj = values.next();
             map.put(adj.getSource(), adj);
         }
-        VLongWritable srcId = getVertexId();
-        map.put(srcId, new AdjacencyListWritable());
 
         // build the vertex list (vertex id in ascending order) and the inverted list of vertexes
         int i = 0;
@@ -86,12 +83,14 @@
             invertedMap.put(v, i++);
         }
 
-        //clean up adjacency list --- remove vertexes who are not neighbors of key
+        // clean up adjacency list --- remove vertexes who are not neighbors of
+        // key
         for (AdjacencyListWritable adj : map.values()) {
             adj.cleanNonMatch(vertexList);
         }
 
-        // get the h-index of the subgraph --- which is the maximum depth to explore
+        // get the h-index of the subgraph --- which is the maximum depth to
+        // explore
         int[] neighborCounts = new int[map.size()];
         i = 0;
         for (AdjacencyListWritable adj : map.values()) {
@@ -105,11 +104,13 @@
             }
             h++;
         }
-        if (h < largestCliqueSizeSoFar) {
+
+        // the clique size is upper-bounded by h+1
+        if (h + 1 < largestCliqueSizeSoFar) {
             return;
         }
 
-        //start depth-first search
+        // start depth-first search
         BitSet cliqueSoFar = new BitSet(h);
         for (VLongWritable v : vertexList) {
             cliqueSoFar.set(invertedMap.get(v));
@@ -117,16 +118,15 @@
             cliqueSoFar.clear();
         }
 
-        //output local maximal cliques
+        // output local maximal cliques
+        tmpValue.setSrcId(getVertexId());
         for (BitSet clique : currentMaximalCliques) {
-            int keyIndex = invertedMap.get(srcId);
-            clique.set(keyIndex);
             generateClique(clique);
-            tmpValue.addCliques(cliques);
+            tmpValue.addCliques(cliqueList);
             tmpValue.setCliqueSize(clique.cardinality());
         }
 
-        //update the vertex state
+        // update the vertex state
         setVertexValue(tmpValue);
     }
 
@@ -136,13 +136,15 @@
      * @param clique
      *            the bitmap representation of a clique
      */
-    private void generateClique(BitSet clique) {
+    private List<VLongWritable> generateClique(BitSet clique) {
+        cliqueList.clear();
         for (int j = 0; j < clique.length();) {
             j = clique.nextSetBit(j);
             VLongWritable v = vertexList.get(j);
-            cliques.add(v);
+            cliqueList.add(v);
             j++;
         }
+        return cliqueList;
     }
 
     /**
@@ -170,7 +172,7 @@
         while (neighbors.hasNext()) {
             VLongWritable neighbor = neighbors.next();
             if (!isTested(neighbor, cliqueSoFar) && isClique(neighbor, cliqueSoFar)) {
-                //snapshot the clique
+                // snapshot the clique
                 int cliqueLength = cliqueSoFar.length();
                 // expand the clique
                 cliqueSoFar.set(invertedMap.get(neighbor));
@@ -217,7 +219,8 @@
         int largestSetIndex = cliqueSoFar.length() - 1;
         if (index > largestSetIndex) {
             // we only return cliques with vertexes in the ascending order
-            // hence, the new vertex must be larger than the largesetSetIndex in the clique
+            // hence, the new vertex must be larger than the largesetSetIndex in
+            // the clique
             return false;
         } else {
             // otherwise, we think the vertex is "tested"
@@ -236,7 +239,8 @@
      */
     private boolean isClique(VLongWritable newVertex, BitSet cliqueSoFar) {
         AdjacencyListWritable adj = map.get(newVertex);
-        // check whether each existing vertex is in the neighbor set of newVertex
+        // check whether each existing vertex is in the neighbor set of
+        // newVertex
         for (int i = 0; i < cliqueSoFar.length();) {
             i = cliqueSoFar.nextSetBit(i);
             VLongWritable v = vertexList.get(i);
@@ -249,9 +253,8 @@
     }
 
     /**
-     * For superstep 1, send outgoing mesages.
-     * For superstep 2, calculate maximal cliques.
-     * otherwise, vote to halt.
+     * For superstep 1, send outgoing mesages. For superstep 2, calculate
+     * maximal cliques. otherwise, vote to halt.
      */
     @Override
     public void compute(Iterator<AdjacencyListWritable> msgIterator) {
@@ -300,9 +303,11 @@
     private void sendOutgoingMsgs(List<Edge<VLongWritable, NullWritable>> edges) {
         for (int i = 0; i < edges.size(); i++) {
             if (edges.get(i).getDestVertexId().get() < getVertexId().get()) {
-                // only add emit for the vertexes whose id is smaller than the vertex id 
+                // only add emit for the vertexes whose id is smaller than the
+                // vertex id
                 // to avoid the duplicate removal step,
-                // because all the resulting cliques will have vertexes in the ascending order.
+                // because all the resulting cliques will have vertexes in the
+                // ascending order.
                 AdjacencyListWritable msg = new AdjacencyListWritable();
                 msg.setSource(getVertexId());
                 for (int j = i + 1; j < edges.size(); j++) {
diff --git a/pregelix/pregelix-example/src/test/java/edu/uci/ics/pregelix/example/jobgen/JobGenerator.java b/pregelix/pregelix-example/src/test/java/edu/uci/ics/pregelix/example/jobgen/JobGenerator.java
index ca5a1c4..0a5b214 100644
--- a/pregelix/pregelix-example/src/test/java/edu/uci/ics/pregelix/example/jobgen/JobGenerator.java
+++ b/pregelix/pregelix-example/src/test/java/edu/uci/ics/pregelix/example/jobgen/JobGenerator.java
@@ -55,6 +55,8 @@
     private static String HDFS_OUTPUTPAH2 = "/resultcomplex";
 
     private static String HDFS_INPUTPATH3 = "/clique";
+    private static String HDFS_INPUTPATH4 = "/clique2";
+    private static String HDFS_INPUTPATH5 = "/clique3";
     private static String HDFS_OUTPUTPAH3 = "/resultclique";
 
     private static void generatePageRankJobReal(String jobName, String outputPath) throws IOException {
@@ -218,6 +220,30 @@
         FileOutputFormat.setOutputPath(job, new Path(HDFS_OUTPUTPAH3));
         job.getConfiguration().writeXml(new FileOutputStream(new File(outputPath)));
     }
+    
+    private static void generateMaximalCliqueJob2(String jobName, String outputPath) throws IOException {
+        PregelixJob job = new PregelixJob(jobName);
+        job.setVertexClass(MaximalCliqueVertex.class);
+        job.setGlobalAggregatorClass(MaximalCliqueAggregator.class);
+        job.setDynamicVertexValueSize(true);
+        job.setVertexInputFormatClass(TextMaximalCliqueInputFormat.class);
+        job.setVertexOutputFormatClass(MaximalCliqueVertexOutputFormat.class);
+        FileInputFormat.setInputPaths(job, HDFS_INPUTPATH4);
+        FileOutputFormat.setOutputPath(job, new Path(HDFS_OUTPUTPAH3));
+        job.getConfiguration().writeXml(new FileOutputStream(new File(outputPath)));
+    }
+    
+    private static void generateMaximalCliqueJob3(String jobName, String outputPath) throws IOException {
+        PregelixJob job = new PregelixJob(jobName);
+        job.setVertexClass(MaximalCliqueVertex.class);
+        job.setGlobalAggregatorClass(MaximalCliqueAggregator.class);
+        job.setDynamicVertexValueSize(true);
+        job.setVertexInputFormatClass(TextMaximalCliqueInputFormat.class);
+        job.setVertexOutputFormatClass(MaximalCliqueVertexOutputFormat.class);
+        FileInputFormat.setInputPaths(job, HDFS_INPUTPATH5);
+        FileOutputFormat.setOutputPath(job, new Path(HDFS_OUTPUTPAH3));
+        job.getConfiguration().writeXml(new FileOutputStream(new File(outputPath)));
+    }
 
     private static void generateGraphMutationJob(String jobName, String outputPath) throws IOException {
         PregelixJob job = new PregelixJob(jobName);
@@ -261,6 +287,8 @@
 
     private static void genMaximalClique() throws IOException {
         generateMaximalCliqueJob("Maximal Clique", outputBase + "MaximalClique.xml");
+        generateMaximalCliqueJob2("Maximal Clique 2", outputBase + "MaximalClique2.xml");
+        generateMaximalCliqueJob3("Maximal Clique 3", outputBase + "MaximalClique3.xml");
     }
 
     private static void genGraphMutation() throws IOException {
diff --git a/pregelix/pregelix-example/src/test/java/edu/uci/ics/pregelix/example/jobrun/RunJobTestCase.java b/pregelix/pregelix-example/src/test/java/edu/uci/ics/pregelix/example/jobrun/RunJobTestCase.java
index 5a556fa..00f6f54 100644
--- a/pregelix/pregelix-example/src/test/java/edu/uci/ics/pregelix/example/jobrun/RunJobTestCase.java
+++ b/pregelix/pregelix-example/src/test/java/edu/uci/ics/pregelix/example/jobrun/RunJobTestCase.java
@@ -25,20 +25,13 @@
 import org.apache.hadoop.mapreduce.lib.output.FileOutputFormat;
 import org.junit.Test;
 
-import edu.uci.ics.hyracks.api.job.JobSpecification;
 import edu.uci.ics.pregelix.api.job.PregelixJob;
-import edu.uci.ics.pregelix.core.jobgen.JobGen;
-import edu.uci.ics.pregelix.core.jobgen.JobGenInnerJoin;
-import edu.uci.ics.pregelix.core.jobgen.JobGenOuterJoin;
-import edu.uci.ics.pregelix.core.jobgen.JobGenOuterJoinSingleSort;
-import edu.uci.ics.pregelix.core.jobgen.JobGenOuterJoinSort;
+import edu.uci.ics.pregelix.core.base.IDriver.Plan;
+import edu.uci.ics.pregelix.core.driver.Driver;
 import edu.uci.ics.pregelix.core.util.PregelixHyracksIntegrationUtil;
-import edu.uci.ics.pregelix.dataflow.util.IterationUtils;
 import edu.uci.ics.pregelix.example.util.TestUtils;
 
 public class RunJobTestCase extends TestCase {
-    private static final String NC1 = "nc1";
-    private static final String HYRACKS_APP_NAME = "pregelix";
     private static String HDFS_INPUTPATH = "/webmap";
     private static String HDFS_OUTPUTPAH = "/result";
 
@@ -48,14 +41,21 @@
     private static String HDFS_INPUTPATH3 = "/clique";
     private static String HDFS_OUTPUTPAH3 = "/resultclique";
 
-    private final PregelixJob job;
-    private JobGen[] giraphJobGens;
-    private final String resultFileName;
-    private final String expectedFileName;
-    private final String jobFile;
+    private static String HDFS_INPUTPATH4 = "/clique2";
+    private static String HDFS_OUTPUTPAH4 = "/resultclique";
 
-    public RunJobTestCase(String hadoopConfPath, String jobName, String jobFile, String resultFile, String expectedFile)
-            throws Exception {
+    private static String HDFS_INPUTPATH5 = "/clique3";
+    private static String HDFS_OUTPUTPAH5 = "/resultclique";
+
+    private final PregelixJob job;
+    private final String resultFileDir;
+    private final String expectedFileDir;
+    private final String jobFile;
+    private final Driver driver = new Driver(this.getClass());
+    private final FileSystem dfs;
+
+    public RunJobTestCase(String hadoopConfPath, String jobName, String jobFile, String resultFile,
+            String expectedFile, FileSystem dfs) throws Exception {
         super("test");
         this.jobFile = jobFile;
         this.job = new PregelixJob("test");
@@ -68,21 +68,20 @@
         } else if (inputPaths[0].toString().endsWith(HDFS_INPUTPATH2)) {
             FileInputFormat.setInputPaths(job, HDFS_INPUTPATH2);
             FileOutputFormat.setOutputPath(job, new Path(HDFS_OUTPUTPAH2));
-        } else {
+        } else if (inputPaths[0].toString().endsWith(HDFS_INPUTPATH3)) {
             FileInputFormat.setInputPaths(job, HDFS_INPUTPATH3);
             FileOutputFormat.setOutputPath(job, new Path(HDFS_OUTPUTPAH3));
+        } else if (inputPaths[0].toString().endsWith(HDFS_INPUTPATH4)) {
+            FileInputFormat.setInputPaths(job, HDFS_INPUTPATH4);
+            FileOutputFormat.setOutputPath(job, new Path(HDFS_OUTPUTPAH4));
+        } else if (inputPaths[0].toString().endsWith(HDFS_INPUTPATH5)) {
+            FileInputFormat.setInputPaths(job, HDFS_INPUTPATH5);
+            FileOutputFormat.setOutputPath(job, new Path(HDFS_OUTPUTPAH5));
         }
         job.setJobName(jobName);
-        this.resultFileName = resultFile;
-        this.expectedFileName = expectedFile;
-        giraphJobGens = new JobGen[4];
-        giraphJobGens[0] = new JobGenOuterJoin(job);
-        waitawhile();
-        giraphJobGens[1] = new JobGenInnerJoin(job);
-        waitawhile();
-        giraphJobGens[2] = new JobGenOuterJoinSort(job);
-        waitawhile();
-        giraphJobGens[3] = new JobGenOuterJoinSingleSort(job);
+        this.resultFileDir = resultFile;
+        this.expectedFileDir = expectedFile;
+        this.dfs = dfs;
     }
 
     private void waitawhile() throws InterruptedException {
@@ -94,89 +93,19 @@
     @Test
     public void test() throws Exception {
         setUp();
-        for (JobGen jobGen : giraphJobGens) {
-            FileSystem dfs = FileSystem.get(job.getConfiguration());
-            dfs.delete(new Path(HDFS_OUTPUTPAH), true);
-            runCreate(jobGen);
-            runDataLoad(jobGen);
-            int i = 1;
-            boolean terminate = false;
-            do {
-                runLoopBodyIteration(jobGen, i);
-                terminate = IterationUtils.readTerminationState(job.getConfiguration(), jobGen.getJobId());
-                i++;
-            } while (!terminate);
-            runIndexScan(jobGen);
-            runHDFSWRite(jobGen);
-            runCleanup(jobGen);
-            compareResults();
+        Plan[] plans = new Plan[] { Plan.OUTER_JOIN };
+        for (Plan plan : plans) {
+            driver.runJob(job, plan, PregelixHyracksIntegrationUtil.CC_HOST,
+                    PregelixHyracksIntegrationUtil.TEST_HYRACKS_CC_CLIENT_PORT, false);
         }
+        compareResults();
         tearDown();
         waitawhile();
     }
 
-    private void runCreate(JobGen jobGen) throws Exception {
-        try {
-            JobSpecification treeCreateJobSpec = jobGen.generateCreatingJob();
-            PregelixHyracksIntegrationUtil.runJob(treeCreateJobSpec, HYRACKS_APP_NAME);
-        } catch (Exception e) {
-            throw e;
-        }
-    }
-
-    private void runDataLoad(JobGen jobGen) throws Exception {
-        try {
-            JobSpecification bulkLoadJobSpec = jobGen.generateLoadingJob();
-            PregelixHyracksIntegrationUtil.runJob(bulkLoadJobSpec, HYRACKS_APP_NAME);
-        } catch (Exception e) {
-            throw e;
-        }
-    }
-
-    private void runLoopBodyIteration(JobGen jobGen, int iteration) throws Exception {
-        try {
-            JobSpecification loopBody = jobGen.generateJob(iteration);
-            PregelixHyracksIntegrationUtil.runJob(loopBody, HYRACKS_APP_NAME);
-        } catch (Exception e) {
-            throw e;
-        }
-    }
-
-    private void runIndexScan(JobGen jobGen) throws Exception {
-        try {
-            JobSpecification scanSortPrintJobSpec = jobGen.scanIndexPrintGraph(NC1, resultFileName);
-            PregelixHyracksIntegrationUtil.runJob(scanSortPrintJobSpec, HYRACKS_APP_NAME);
-        } catch (Exception e) {
-            throw e;
-        }
-    }
-
-    private void runHDFSWRite(JobGen jobGen) throws Exception {
-        try {
-            JobSpecification scanSortPrintJobSpec = jobGen.scanIndexWriteGraph();
-            PregelixHyracksIntegrationUtil.runJob(scanSortPrintJobSpec, HYRACKS_APP_NAME);
-        } catch (Exception e) {
-            throw e;
-        }
-    }
-
-    private void runCleanup(JobGen jobGen) throws Exception {
-        try {
-            JobSpecification[] cleanups = jobGen.generateCleanup();
-            runJobArray(cleanups);
-        } catch (Exception e) {
-            throw e;
-        }
-    }
-
-    private void runJobArray(JobSpecification[] jobs) throws Exception {
-        for (JobSpecification job : jobs) {
-            PregelixHyracksIntegrationUtil.runJob(job, HYRACKS_APP_NAME);
-        }
-    }
-
     private void compareResults() throws Exception {
-        TestUtils.compareWithResult(new File(resultFileName), new File(expectedFileName));
+        dfs.copyToLocalFile(FileOutputFormat.getOutputPath(job), new Path(resultFileDir));
+        TestUtils.compareWithResultDir(new File(expectedFileDir), new File(resultFileDir));
     }
 
     public String toString() {
diff --git a/pregelix/pregelix-example/src/test/java/edu/uci/ics/pregelix/example/jobrun/RunJobTestSuite.java b/pregelix/pregelix-example/src/test/java/edu/uci/ics/pregelix/example/jobrun/RunJobTestSuite.java
index 4bf83e6..87bc40d 100644
--- a/pregelix/pregelix-example/src/test/java/edu/uci/ics/pregelix/example/jobrun/RunJobTestSuite.java
+++ b/pregelix/pregelix-example/src/test/java/edu/uci/ics/pregelix/example/jobrun/RunJobTestSuite.java
@@ -62,7 +62,12 @@
     private static final String DATA_PATH3 = "data/clique/clique.txt";
     private static final String HDFS_PATH3 = "/clique/";
 
-    private static final String HYRACKS_APP_NAME = "pregelix";
+    private static final String DATA_PATH4 = "data/clique2/clique.txt";
+    private static final String HDFS_PATH4 = "/clique2/";
+
+    private static final String DATA_PATH5 = "data/clique3/clique.txt";
+    private static final String HDFS_PATH5 = "/clique3/";
+
     private static final String HADOOP_CONF_PATH = ACTUAL_RESULT_DIR + File.separator + "conf.xml";
     private MiniDFSCluster dfsCluster;
 
@@ -111,6 +116,16 @@
         dfs.mkdirs(dest);
         dfs.copyFromLocalFile(src, dest);
 
+        src = new Path(DATA_PATH4);
+        dest = new Path(HDFS_PATH4);
+        dfs.mkdirs(dest);
+        dfs.copyFromLocalFile(src, dest);
+
+        src = new Path(DATA_PATH5);
+        dest = new Path(HDFS_PATH5);
+        dfs.mkdirs(dest);
+        dfs.copyFromLocalFile(src, dest);
+
         DataOutputStream confOutput = new DataOutputStream(new FileOutputStream(new File(HADOOP_CONF_PATH)));
         conf.writeXml(confOutput);
         confOutput.flush();
@@ -138,6 +153,7 @@
         RunJobTestSuite testSuite = new RunJobTestSuite();
         testSuite.setUp();
         boolean onlyEnabled = false;
+        FileSystem dfs = FileSystem.get(testSuite.conf);
 
         if (onlys.size() > 0) {
             onlyEnabled = true;
@@ -153,7 +169,7 @@
                     String resultFileName = ACTUAL_RESULT_DIR + File.separator + jobExtToResExt(qFile.getName());
                     String expectedFileName = EXPECTED_RESULT_DIR + File.separator + jobExtToResExt(qFile.getName());
                     testSuite.addTest(new RunJobTestCase(HADOOP_CONF_PATH, qFile.getName(), qFile.getAbsolutePath()
-                            .toString(), resultFileName, expectedFileName));
+                            .toString(), resultFileName, expectedFileName, dfs));
                 }
             }
         }
@@ -193,7 +209,7 @@
 
     private static String jobExtToResExt(String fname) {
         int dot = fname.lastIndexOf('.');
-        return fname.substring(0, dot + 1) + FILE_EXTENSION_OF_RESULTS;
+        return fname.substring(0, dot);
     }
 
     private static boolean isInList(List<String> onlys, String name) {
diff --git a/pregelix/pregelix-example/src/test/java/edu/uci/ics/pregelix/example/util/TestUtils.java b/pregelix/pregelix-example/src/test/java/edu/uci/ics/pregelix/example/util/TestUtils.java
index d89ec46..d406125 100644
--- a/pregelix/pregelix-example/src/test/java/edu/uci/ics/pregelix/example/util/TestUtils.java
+++ b/pregelix/pregelix-example/src/test/java/edu/uci/ics/pregelix/example/util/TestUtils.java
@@ -20,6 +20,13 @@
 
 public class TestUtils {
 
+    public static void compareWithResultDir(File expectedFileDir, File actualFileDir) throws Exception {
+        String[] fileNames = expectedFileDir.list();
+        for (String fileName : fileNames) {
+            compareWithResult(new File(expectedFileDir, fileName), new File(actualFileDir, fileName));
+        }
+    }
+
     public static void compareWithResult(File expectedFile, File actualFile) throws Exception {
         BufferedReader readerExpected = new BufferedReader(new FileReader(expectedFile));
         BufferedReader readerActual = new BufferedReader(new FileReader(actualFile));
@@ -28,7 +35,6 @@
         try {
             while ((lineExpected = readerExpected.readLine()) != null) {
                 lineActual = readerActual.readLine();
-                // Assert.assertEquals(lineExpected, lineActual);
                 if (lineActual == null) {
                     throw new Exception("Actual result changed at line " + num + ":\n< " + lineExpected + "\n> ");
                 }
@@ -62,8 +68,10 @@
             if (row1.equals(row2))
                 continue;
 
-            String[] fields1 = row1.split(" ");
-            String[] fields2 = row2.split(" ");
+            boolean spaceOrTab = false;
+            spaceOrTab = row1.contains(" ");
+            String[] fields1 = spaceOrTab ? row1.split(" ") : row1.split("\t");
+            String[] fields2 = spaceOrTab ? row2.split(" ") : row2.split("\t");
 
             for (int j = 0; j < fields1.length; j++) {
                 if (fields1[j].equals(fields2[j])) {
@@ -76,7 +84,7 @@
                     float float1 = (float) double1.doubleValue();
                     float float2 = (float) double2.doubleValue();
 
-                    if (Math.abs(float1 - float2) == 0)
+                    if (Math.abs(float1 - float2) < 1.0e-7)
                         continue;
                     else {
                         return false;
diff --git a/pregelix/pregelix-example/src/test/resources/cluster/stores.properties b/pregelix/pregelix-example/src/test/resources/cluster/stores.properties
index daf881e..04732be 100644
--- a/pregelix/pregelix-example/src/test/resources/cluster/stores.properties
+++ b/pregelix/pregelix-example/src/test/resources/cluster/stores.properties
@@ -1 +1 @@
-store=teststore
\ No newline at end of file
+store=teststore1,teststore2
\ No newline at end of file
diff --git a/pregelix/pregelix-example/src/test/resources/expected/ConnectedComponentsReal.result b/pregelix/pregelix-example/src/test/resources/expected/ConnectedComponentsReal.result
deleted file mode 100644
index 45376e2..0000000
--- a/pregelix/pregelix-example/src/test/resources/expected/ConnectedComponentsReal.result
+++ /dev/null
@@ -1,20 +0,0 @@
-0 0
-1 0
-2 0
-3 0
-4 0
-5 0
-6 0
-7 0
-8 0
-9 0
-10 0
-11 0
-12 0
-13 0
-14 0
-15 0
-16 0
-17 0
-18 0
-19 0
diff --git a/pregelix/pregelix-example/src/test/resources/expected/ConnectedComponentsReal/part-0 b/pregelix/pregelix-example/src/test/resources/expected/ConnectedComponentsReal/part-0
new file mode 100755
index 0000000..f1f1d9b
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/ConnectedComponentsReal/part-0
@@ -0,0 +1,5 @@
+0	0
+4	0
+8	0
+12	0
+16	0
diff --git a/pregelix/pregelix-example/src/test/resources/expected/ConnectedComponentsReal/part-1 b/pregelix/pregelix-example/src/test/resources/expected/ConnectedComponentsReal/part-1
new file mode 100755
index 0000000..0fa02c1
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/ConnectedComponentsReal/part-1
@@ -0,0 +1,5 @@
+1	0
+5	0
+9	0
+13	0
+17	0
diff --git a/pregelix/pregelix-example/src/test/resources/expected/ConnectedComponentsReal/part-2 b/pregelix/pregelix-example/src/test/resources/expected/ConnectedComponentsReal/part-2
new file mode 100755
index 0000000..542ccae
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/ConnectedComponentsReal/part-2
@@ -0,0 +1,5 @@
+2	0
+6	0
+10	0
+14	0
+18	0
diff --git a/pregelix/pregelix-example/src/test/resources/expected/ConnectedComponentsReal/part-3 b/pregelix/pregelix-example/src/test/resources/expected/ConnectedComponentsReal/part-3
new file mode 100755
index 0000000..1d5d6d9
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/ConnectedComponentsReal/part-3
@@ -0,0 +1,5 @@
+3	0
+7	0
+11	0
+15	0
+19	0
diff --git a/pregelix/pregelix-example/src/test/resources/expected/ConnectedComponentsRealComplex.result b/pregelix/pregelix-example/src/test/resources/expected/ConnectedComponentsRealComplex.result
deleted file mode 100644
index dbc30fc..0000000
--- a/pregelix/pregelix-example/src/test/resources/expected/ConnectedComponentsRealComplex.result
+++ /dev/null
@@ -1,23 +0,0 @@
-0 0
-1 0
-2 0
-3 0
-4 0
-5 0
-6 0
-7 0
-8 0
-9 0
-10 0
-11 0
-12 0
-13 0
-14 0
-15 0
-16 0
-17 0
-18 0
-19 0
-21 21
-25 25
-27 27
diff --git a/pregelix/pregelix-example/src/test/resources/expected/ConnectedComponentsRealComplex/part-0 b/pregelix/pregelix-example/src/test/resources/expected/ConnectedComponentsRealComplex/part-0
new file mode 100755
index 0000000..f1f1d9b
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/ConnectedComponentsRealComplex/part-0
@@ -0,0 +1,5 @@
+0	0
+4	0
+8	0
+12	0
+16	0
diff --git a/pregelix/pregelix-example/src/test/resources/expected/ConnectedComponentsRealComplex/part-1 b/pregelix/pregelix-example/src/test/resources/expected/ConnectedComponentsRealComplex/part-1
new file mode 100755
index 0000000..4e7d87a
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/ConnectedComponentsRealComplex/part-1
@@ -0,0 +1,7 @@
+1	0
+5	0
+9	0
+13	0
+17	0
+21	21
+25	25
diff --git a/pregelix/pregelix-example/src/test/resources/expected/ConnectedComponentsRealComplex/part-2 b/pregelix/pregelix-example/src/test/resources/expected/ConnectedComponentsRealComplex/part-2
new file mode 100755
index 0000000..542ccae
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/ConnectedComponentsRealComplex/part-2
@@ -0,0 +1,5 @@
+2	0
+6	0
+10	0
+14	0
+18	0
diff --git a/pregelix/pregelix-example/src/test/resources/expected/ConnectedComponentsRealComplex/part-3 b/pregelix/pregelix-example/src/test/resources/expected/ConnectedComponentsRealComplex/part-3
new file mode 100755
index 0000000..513f3ff
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/ConnectedComponentsRealComplex/part-3
@@ -0,0 +1,6 @@
+3	0
+7	0
+11	0
+15	0
+19	0
+27	27
diff --git a/pregelix/pregelix-example/src/test/resources/expected/GraphMutation.result b/pregelix/pregelix-example/src/test/resources/expected/GraphMutation.result
deleted file mode 100644
index a30166c..0000000
--- a/pregelix/pregelix-example/src/test/resources/expected/GraphMutation.result
+++ /dev/null
@@ -1,13 +0,0 @@
-1 0.0
-5 0.0
-7 0.0
-11 0.0
-13 0.0
-17 0.0
-19 0.0
-100 0.0
-500 0.0
-700 0.0
-1100 0.0
-1300 0.0
-1700 0.0
diff --git a/pregelix/pregelix-example/src/test/resources/expected/GraphMutation/part-0 b/pregelix/pregelix-example/src/test/resources/expected/GraphMutation/part-0
new file mode 100755
index 0000000..b5f7ed3
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/GraphMutation/part-0
@@ -0,0 +1,6 @@
+100	0.0
+500	0.0
+700	0.0
+1100	0.0
+1300	0.0
+1700	0.0
diff --git a/pregelix/pregelix-example/src/test/resources/expected/GraphMutation/part-1 b/pregelix/pregelix-example/src/test/resources/expected/GraphMutation/part-1
new file mode 100755
index 0000000..4eca51d
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/GraphMutation/part-1
@@ -0,0 +1,4 @@
+1	0.0
+5	0.0
+13	0.0
+17	0.0
diff --git a/pregelix/pregelix-example/src/test/resources/expected/GraphMutation/part-2 b/pregelix/pregelix-example/src/test/resources/expected/GraphMutation/part-2
new file mode 100755
index 0000000..e69de29
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/GraphMutation/part-2
diff --git a/pregelix/pregelix-example/src/test/resources/expected/GraphMutation/part-3 b/pregelix/pregelix-example/src/test/resources/expected/GraphMutation/part-3
new file mode 100755
index 0000000..9446ff7
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/GraphMutation/part-3
@@ -0,0 +1,3 @@
+7	0.0
+11	0.0
+19	0.0
diff --git a/pregelix/pregelix-example/src/test/resources/expected/MaximalClique.result b/pregelix/pregelix-example/src/test/resources/expected/MaximalClique.result
deleted file mode 100644
index d238037..0000000
--- a/pregelix/pregelix-example/src/test/resources/expected/MaximalClique.result
+++ /dev/null
@@ -1,7 +0,0 @@
-1 1,2,3,4;
-2 2,3,4;
-3 
-4 
-5 
-6 
-7 
diff --git a/pregelix/pregelix-example/src/test/resources/expected/MaximalClique/part-0 b/pregelix/pregelix-example/src/test/resources/expected/MaximalClique/part-0
new file mode 100755
index 0000000..902fadf
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/MaximalClique/part-0
@@ -0,0 +1 @@
+4	
diff --git a/pregelix/pregelix-example/src/test/resources/expected/MaximalClique/part-1 b/pregelix/pregelix-example/src/test/resources/expected/MaximalClique/part-1
new file mode 100755
index 0000000..ba34424
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/MaximalClique/part-1
@@ -0,0 +1,2 @@
+1	1,2,3,4;
+5	
diff --git a/pregelix/pregelix-example/src/test/resources/expected/MaximalClique/part-2 b/pregelix/pregelix-example/src/test/resources/expected/MaximalClique/part-2
new file mode 100755
index 0000000..834e389
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/MaximalClique/part-2
@@ -0,0 +1,2 @@
+2	2,3,4;
+6	
diff --git a/pregelix/pregelix-example/src/test/resources/expected/MaximalClique/part-3 b/pregelix/pregelix-example/src/test/resources/expected/MaximalClique/part-3
new file mode 100755
index 0000000..b8e2461
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/MaximalClique/part-3
@@ -0,0 +1,2 @@
+3	3,4;
+7	
diff --git a/pregelix/pregelix-example/src/test/resources/expected/MaximalClique2/part-0 b/pregelix/pregelix-example/src/test/resources/expected/MaximalClique2/part-0
new file mode 100755
index 0000000..902fadf
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/MaximalClique2/part-0
@@ -0,0 +1 @@
+4	
diff --git a/pregelix/pregelix-example/src/test/resources/expected/MaximalClique2/part-1 b/pregelix/pregelix-example/src/test/resources/expected/MaximalClique2/part-1
new file mode 100755
index 0000000..b83e1a3
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/MaximalClique2/part-1
@@ -0,0 +1,2 @@
+1	1,2,3;1,3,4;
+5	
diff --git a/pregelix/pregelix-example/src/test/resources/expected/MaximalClique2/part-2 b/pregelix/pregelix-example/src/test/resources/expected/MaximalClique2/part-2
new file mode 100755
index 0000000..45e2a23
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/MaximalClique2/part-2
@@ -0,0 +1,2 @@
+2	2,3;
+6	
diff --git a/pregelix/pregelix-example/src/test/resources/expected/MaximalClique2/part-3 b/pregelix/pregelix-example/src/test/resources/expected/MaximalClique2/part-3
new file mode 100755
index 0000000..447aa38
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/MaximalClique2/part-3
@@ -0,0 +1 @@
+3	3,4;3,5;
diff --git a/pregelix/pregelix-example/src/test/resources/expected/MaximalClique3/part-0 b/pregelix/pregelix-example/src/test/resources/expected/MaximalClique3/part-0
new file mode 100755
index 0000000..b4dced4
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/MaximalClique3/part-0
@@ -0,0 +1,5 @@
+0	0,19;
+4	4,5,6,7,8,9;
+8	
+12	12,13,14,15,16,17,18,19;
+16	
diff --git a/pregelix/pregelix-example/src/test/resources/expected/MaximalClique3/part-1 b/pregelix/pregelix-example/src/test/resources/expected/MaximalClique3/part-1
new file mode 100755
index 0000000..f554dfe
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/MaximalClique3/part-1
@@ -0,0 +1,5 @@
+1	1,2,3,4,5,6,7,8,9;
+5	
+9	
+13	
+17	
diff --git a/pregelix/pregelix-example/src/test/resources/expected/MaximalClique3/part-2 b/pregelix/pregelix-example/src/test/resources/expected/MaximalClique3/part-2
new file mode 100755
index 0000000..4df9b8c
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/MaximalClique3/part-2
@@ -0,0 +1,5 @@
+2	2,3,4,5,6,7,8,9;
+6	
+10	
+14	
+18	
diff --git a/pregelix/pregelix-example/src/test/resources/expected/MaximalClique3/part-3 b/pregelix/pregelix-example/src/test/resources/expected/MaximalClique3/part-3
new file mode 100755
index 0000000..5131560
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/MaximalClique3/part-3
@@ -0,0 +1,5 @@
+3	3,4,5,6,7,8,9;
+7	
+11	11,12,13,14,15,16,17,18,19;
+15	
+19	
diff --git a/pregelix/pregelix-example/src/test/resources/expected/PageRank.result b/pregelix/pregelix-example/src/test/resources/expected/PageRank.result
deleted file mode 100644
index 9c4d83a..0000000
--- a/pregelix/pregelix-example/src/test/resources/expected/PageRank.result
+++ /dev/null
@@ -1,20 +0,0 @@
-0 0.008290140026154316
-1 0.1535152819247165
-2 0.14646839195826475
-3 0.08125113985998214
-4 0.03976979906329426
-5 0.0225041581462058
-6 0.015736276824953852
-7 0.012542224114863661
-8 0.010628239626209894
-9 0.009294348455354817
-10 0.008290140026154316
-11 0.15351528192471647
-12 0.14646839195826472
-13 0.08125113985998214
-14 0.03976979906329425
-15 0.0225041581462058
-16 0.015736276824953852
-17 0.012542224114863661
-18 0.010628239626209894
-19 0.009294348455354817
diff --git a/pregelix/pregelix-example/src/test/resources/expected/PageRank/part-0 b/pregelix/pregelix-example/src/test/resources/expected/PageRank/part-0
new file mode 100755
index 0000000..5b07add
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/PageRank/part-0
@@ -0,0 +1,10 @@
+0	0.008290140026154316
+4	0.039769799063294246
+8	0.010628239626209894
+12	0.14646839195826478
+16	0.015736276824953852
+20	0.008290140026154316
+24	0.039769799063294246
+28	0.010628239626209894
+32	0.14646839195826478
+36	0.015736276824953852
diff --git a/pregelix/pregelix-example/src/test/resources/expected/PageRank/part-1 b/pregelix/pregelix-example/src/test/resources/expected/PageRank/part-1
new file mode 100755
index 0000000..caf8a4c
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/PageRank/part-1
@@ -0,0 +1,10 @@
+1	0.15351528192471647
+5	0.0225041581462058
+9	0.009294348455354817
+13	0.08125113985998214
+17	0.012542224114863661
+21	0.15351528192471647
+25	0.0225041581462058
+29	0.009294348455354817
+33	0.08125113985998214
+37	0.012542224114863661
diff --git a/pregelix/pregelix-example/src/test/resources/expected/PageRank/part-2 b/pregelix/pregelix-example/src/test/resources/expected/PageRank/part-2
new file mode 100755
index 0000000..29a9d52
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/PageRank/part-2
@@ -0,0 +1,10 @@
+2	0.14646839195826475
+6	0.015736276824953852
+10	0.008290140026154316
+14	0.03976979906329426
+18	0.010628239626209894
+22	0.14646839195826472
+26	0.01573627682495385
+30	0.008290140026154316
+34	0.03976979906329426
+38	0.010628239626209894
diff --git a/pregelix/pregelix-example/src/test/resources/expected/PageRank/part-3 b/pregelix/pregelix-example/src/test/resources/expected/PageRank/part-3
new file mode 100755
index 0000000..c9bfeb8
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/PageRank/part-3
@@ -0,0 +1,10 @@
+3	0.08125113985998214
+7	0.012542224114863663
+11	0.1535152819247165
+15	0.0225041581462058
+19	0.009294348455354817
+23	0.08125113985998211
+27	0.012542224114863661
+31	0.1535152819247165
+35	0.0225041581462058
+39	0.009294348455354817
diff --git a/pregelix/pregelix-example/src/test/resources/expected/PageRankReal.result b/pregelix/pregelix-example/src/test/resources/expected/PageRankReal.result
deleted file mode 100644
index 6432eda..0000000
--- a/pregelix/pregelix-example/src/test/resources/expected/PageRankReal.result
+++ /dev/null
@@ -1,20 +0,0 @@
-0 0.008290140026154316
-1 0.1535152819247165
-2 0.14646839195826475
-3 0.08125113985998214
-4 0.03976979906329426
-5 0.0225041581462058
-6 0.015736276824953852
-7 0.012542224114863661
-8 0.010628239626209894
-9 0.009294348455354817
-10 0.008290140026154316
-11 0.15351528192471647
-12 0.14646839195826472
-13 0.08125113985998214
-14 0.03976979906329426
-15 0.0225041581462058
-16 0.015736276824953852
-17 0.012542224114863661
-18 0.010628239626209894
-19 0.009294348455354817
diff --git a/pregelix/pregelix-example/src/test/resources/expected/PageRankReal/part-0 b/pregelix/pregelix-example/src/test/resources/expected/PageRankReal/part-0
new file mode 100755
index 0000000..383076e
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/PageRankReal/part-0
@@ -0,0 +1,5 @@
+0	0.008290140026154316
+4	0.03976979906329426
+8	0.010628239626209894
+12	0.14646839195826478
+16	0.015736276824953852
diff --git a/pregelix/pregelix-example/src/test/resources/expected/PageRankReal/part-1 b/pregelix/pregelix-example/src/test/resources/expected/PageRankReal/part-1
new file mode 100755
index 0000000..0b1a38c
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/PageRankReal/part-1
@@ -0,0 +1,5 @@
+1	0.15351528192471653
+5	0.0225041581462058
+9	0.009294348455354817
+13	0.08125113985998214
+17	0.012542224114863661
diff --git a/pregelix/pregelix-example/src/test/resources/expected/PageRankReal/part-2 b/pregelix/pregelix-example/src/test/resources/expected/PageRankReal/part-2
new file mode 100755
index 0000000..ec995b2
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/PageRankReal/part-2
@@ -0,0 +1,5 @@
+2	0.14646839195826478
+6	0.015736276824953852
+10	0.008290140026154316
+14	0.03976979906329426
+18	0.010628239626209894
diff --git a/pregelix/pregelix-example/src/test/resources/expected/PageRankReal/part-3 b/pregelix/pregelix-example/src/test/resources/expected/PageRankReal/part-3
new file mode 100755
index 0000000..edb7484
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/PageRankReal/part-3
@@ -0,0 +1,5 @@
+3	0.08125113985998214
+7	0.012542224114863661
+11	0.15351528192471653
+15	0.0225041581462058
+19	0.009294348455354817
diff --git a/pregelix/pregelix-example/src/test/resources/expected/PageRankRealComplex.result b/pregelix/pregelix-example/src/test/resources/expected/PageRankRealComplex.result
deleted file mode 100644
index 2bd09e1..0000000
--- a/pregelix/pregelix-example/src/test/resources/expected/PageRankRealComplex.result
+++ /dev/null
@@ -1,23 +0,0 @@
-0 0.0072088164890121405
-1 0.12352056961948686
-2 0.12045670441668178
-3 0.06798545786459467
-4 0.03387281259892814
-5 0.01942600635480669
-6 0.013661020012182747
-7 0.0109034351563503
-8 0.009241684574402657
-9 0.008082028259564783
-10 0.007208817414047232
-11 0.07555839219845861
-12 0.07249452699565352
-13 0.05063539695954156
-14 0.029644452692487822
-15 0.018670183493927354
-16 0.013558283213067561
-17 0.010892790899883237
-18 0.009240874593661061
-19 0.008081987856433137
-21 0.006521739130434782
-25 0.006521739130434782
-27 0.006521739130434782
diff --git a/pregelix/pregelix-example/src/test/resources/expected/PageRankRealComplex/part-0 b/pregelix/pregelix-example/src/test/resources/expected/PageRankRealComplex/part-0
new file mode 100755
index 0000000..1d26aee2
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/PageRankRealComplex/part-0
@@ -0,0 +1,5 @@
+0	0.0072088164890121405
+4	0.03387281259892814
+8	0.009241684574402657
+12	0.07249452699565351
+16	0.013558283213067561
diff --git a/pregelix/pregelix-example/src/test/resources/expected/PageRankRealComplex/part-1 b/pregelix/pregelix-example/src/test/resources/expected/PageRankRealComplex/part-1
new file mode 100755
index 0000000..d013c3e
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/PageRankRealComplex/part-1
@@ -0,0 +1,7 @@
+1	0.12352056961948689
+5	0.01942600635480669
+9	0.008082028259564783
+13	0.050635396959541557
+17	0.010892790899883237
+21	0.006521739130434782
+25	0.006521739130434782
diff --git a/pregelix/pregelix-example/src/test/resources/expected/PageRankRealComplex/part-2 b/pregelix/pregelix-example/src/test/resources/expected/PageRankRealComplex/part-2
new file mode 100755
index 0000000..e8aa0e1
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/PageRankRealComplex/part-2
@@ -0,0 +1,5 @@
+2	0.12045670441668178
+6	0.013661020012182747
+10	0.007208817414047232
+14	0.029644452692487822
+18	0.009240874593661061
diff --git a/pregelix/pregelix-example/src/test/resources/expected/PageRankRealComplex/part-3 b/pregelix/pregelix-example/src/test/resources/expected/PageRankRealComplex/part-3
new file mode 100755
index 0000000..e52970d
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/PageRankRealComplex/part-3
@@ -0,0 +1,6 @@
+3	0.06798545786459467
+7	0.0109034351563503
+11	0.0755583921984586
+15	0.018670183493927354
+19	0.008081987856433137
+27	0.006521739130434782
diff --git a/pregelix/pregelix-example/src/test/resources/expected/PageRankRealDynamic.result b/pregelix/pregelix-example/src/test/resources/expected/PageRankRealDynamic.result
deleted file mode 100644
index 6432eda..0000000
--- a/pregelix/pregelix-example/src/test/resources/expected/PageRankRealDynamic.result
+++ /dev/null
@@ -1,20 +0,0 @@
-0 0.008290140026154316
-1 0.1535152819247165
-2 0.14646839195826475
-3 0.08125113985998214
-4 0.03976979906329426
-5 0.0225041581462058
-6 0.015736276824953852
-7 0.012542224114863661
-8 0.010628239626209894
-9 0.009294348455354817
-10 0.008290140026154316
-11 0.15351528192471647
-12 0.14646839195826472
-13 0.08125113985998214
-14 0.03976979906329426
-15 0.0225041581462058
-16 0.015736276824953852
-17 0.012542224114863661
-18 0.010628239626209894
-19 0.009294348455354817
diff --git a/pregelix/pregelix-example/src/test/resources/expected/PageRankRealDynamic/part-0 b/pregelix/pregelix-example/src/test/resources/expected/PageRankRealDynamic/part-0
new file mode 100755
index 0000000..383076e
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/PageRankRealDynamic/part-0
@@ -0,0 +1,5 @@
+0	0.008290140026154316
+4	0.03976979906329426
+8	0.010628239626209894
+12	0.14646839195826478
+16	0.015736276824953852
diff --git a/pregelix/pregelix-example/src/test/resources/expected/PageRankRealDynamic/part-1 b/pregelix/pregelix-example/src/test/resources/expected/PageRankRealDynamic/part-1
new file mode 100755
index 0000000..0b1a38c
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/PageRankRealDynamic/part-1
@@ -0,0 +1,5 @@
+1	0.15351528192471653
+5	0.0225041581462058
+9	0.009294348455354817
+13	0.08125113985998214
+17	0.012542224114863661
diff --git a/pregelix/pregelix-example/src/test/resources/expected/PageRankRealDynamic/part-2 b/pregelix/pregelix-example/src/test/resources/expected/PageRankRealDynamic/part-2
new file mode 100755
index 0000000..ec995b2
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/PageRankRealDynamic/part-2
@@ -0,0 +1,5 @@
+2	0.14646839195826478
+6	0.015736276824953852
+10	0.008290140026154316
+14	0.03976979906329426
+18	0.010628239626209894
diff --git a/pregelix/pregelix-example/src/test/resources/expected/PageRankRealDynamic/part-3 b/pregelix/pregelix-example/src/test/resources/expected/PageRankRealDynamic/part-3
new file mode 100755
index 0000000..5593738
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/PageRankRealDynamic/part-3
@@ -0,0 +1,5 @@
+3	0.08125113985998214
+7	0.012542224114863661
+11	0.15351528192471653
+15	0.022504158146205808
+19	0.009294348455354817
diff --git a/pregelix/pregelix-example/src/test/resources/expected/PageRankRealNoCombiner.result b/pregelix/pregelix-example/src/test/resources/expected/PageRankRealNoCombiner.result
deleted file mode 100755
index 9a747a6..0000000
--- a/pregelix/pregelix-example/src/test/resources/expected/PageRankRealNoCombiner.result
+++ /dev/null
@@ -1,20 +0,0 @@
-0 0.008290140026154316
-1 0.15351528192471647
-2 0.14646839195826475
-3 0.08125113985998211
-4 0.03976979906329425
-5 0.0225041581462058
-6 0.01573627682495385
-7 0.012542224114863661
-8 0.010628239626209894
-9 0.009294348455354817
-10 0.008290140026154316
-11 0.1535152819247165
-12 0.14646839195826475
-13 0.08125113985998214
-14 0.03976979906329426
-15 0.0225041581462058
-16 0.015736276824953852
-17 0.012542224114863661
-18 0.010628239626209894
-19 0.009294348455354817
diff --git a/pregelix/pregelix-example/src/test/resources/expected/PageRankRealNoCombiner/part-0 b/pregelix/pregelix-example/src/test/resources/expected/PageRankRealNoCombiner/part-0
new file mode 100755
index 0000000..6fd5f60
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/PageRankRealNoCombiner/part-0
@@ -0,0 +1,5 @@
+0	0.008290140026154316
+4	0.03976979906329425
+8	0.010628239626209894
+12	0.14646839195826475
+16	0.01573627682495385
diff --git a/pregelix/pregelix-example/src/test/resources/expected/PageRankRealNoCombiner/part-1 b/pregelix/pregelix-example/src/test/resources/expected/PageRankRealNoCombiner/part-1
new file mode 100755
index 0000000..0b1a38c
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/PageRankRealNoCombiner/part-1
@@ -0,0 +1,5 @@
+1	0.15351528192471653
+5	0.0225041581462058
+9	0.009294348455354817
+13	0.08125113985998214
+17	0.012542224114863661
diff --git a/pregelix/pregelix-example/src/test/resources/expected/PageRankRealNoCombiner/part-2 b/pregelix/pregelix-example/src/test/resources/expected/PageRankRealNoCombiner/part-2
new file mode 100755
index 0000000..69a2803
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/PageRankRealNoCombiner/part-2
@@ -0,0 +1,5 @@
+2	0.14646839195826478
+6	0.015736276824953852
+10	0.008290140026154316
+14	0.03976979906329425
+18	0.010628239626209894
diff --git a/pregelix/pregelix-example/src/test/resources/expected/PageRankRealNoCombiner/part-3 b/pregelix/pregelix-example/src/test/resources/expected/PageRankRealNoCombiner/part-3
new file mode 100755
index 0000000..7dcb359
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/PageRankRealNoCombiner/part-3
@@ -0,0 +1,5 @@
+3	0.08125113985998214
+7	0.012542224114863661
+11	0.1535152819247165
+15	0.0225041581462058
+19	0.009294348455354817
diff --git a/pregelix/pregelix-example/src/test/resources/expected/ReachibilityRealComplex.result b/pregelix/pregelix-example/src/test/resources/expected/ReachibilityRealComplex.result
deleted file mode 100644
index a1dfc0f..0000000
--- a/pregelix/pregelix-example/src/test/resources/expected/ReachibilityRealComplex.result
+++ /dev/null
@@ -1,23 +0,0 @@
-0 2
-1 3
-2 1
-3 1
-4 1
-5 1
-6 1
-7 1
-8 1
-9 1
-10 3
-11 2
-12 2
-13 2
-14 2
-15 2
-16 2
-17 2
-18 2
-19 2
-21 0
-25 0
-27 0
diff --git a/pregelix/pregelix-example/src/test/resources/expected/ReachibilityRealComplex/part-0 b/pregelix/pregelix-example/src/test/resources/expected/ReachibilityRealComplex/part-0
new file mode 100755
index 0000000..6e3ba89
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/ReachibilityRealComplex/part-0
@@ -0,0 +1,5 @@
+0	0
+4	1
+8	1
+12	2
+16	2
diff --git a/pregelix/pregelix-example/src/test/resources/expected/ReachibilityRealComplex/part-1 b/pregelix/pregelix-example/src/test/resources/expected/ReachibilityRealComplex/part-1
new file mode 100755
index 0000000..2fcc8eb
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/ReachibilityRealComplex/part-1
@@ -0,0 +1,7 @@
+1	1
+5	1
+9	1
+13	2
+17	2
+21	0
+25	0
diff --git a/pregelix/pregelix-example/src/test/resources/expected/ReachibilityRealComplex/part-2 b/pregelix/pregelix-example/src/test/resources/expected/ReachibilityRealComplex/part-2
new file mode 100755
index 0000000..c93732c
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/ReachibilityRealComplex/part-2
@@ -0,0 +1,5 @@
+2	1
+6	1
+10	3
+14	2
+18	2
diff --git a/pregelix/pregelix-example/src/test/resources/expected/ReachibilityRealComplex/part-3 b/pregelix/pregelix-example/src/test/resources/expected/ReachibilityRealComplex/part-3
new file mode 100755
index 0000000..c745349
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/ReachibilityRealComplex/part-3
@@ -0,0 +1,6 @@
+3	1
+7	1
+11	2
+15	2
+19	2
+27	0
diff --git a/pregelix/pregelix-example/src/test/resources/expected/ReachibilityRealComplexNoConnectivity.result b/pregelix/pregelix-example/src/test/resources/expected/ReachibilityRealComplexNoConnectivity.result
deleted file mode 100644
index 1693fb2..0000000
--- a/pregelix/pregelix-example/src/test/resources/expected/ReachibilityRealComplexNoConnectivity.result
+++ /dev/null
@@ -1,23 +0,0 @@
-0 1
-1 1
-2 1
-3 1
-4 1
-5 1
-6 1
-7 1
-8 1
-9 1
-10 1
-11 1
-12 1
-13 1
-14 1
-15 1
-16 1
-17 1
-18 1
-19 1
-21 0
-25 2
-27 0
diff --git a/pregelix/pregelix-example/src/test/resources/expected/ReachibilityRealComplexNoConnectivity/part-0 b/pregelix/pregelix-example/src/test/resources/expected/ReachibilityRealComplexNoConnectivity/part-0
new file mode 100755
index 0000000..8e02e13
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/ReachibilityRealComplexNoConnectivity/part-0
@@ -0,0 +1,5 @@
+0	1
+4	1
+8	1
+12	1
+16	1
diff --git a/pregelix/pregelix-example/src/test/resources/expected/ReachibilityRealComplexNoConnectivity/part-1 b/pregelix/pregelix-example/src/test/resources/expected/ReachibilityRealComplexNoConnectivity/part-1
new file mode 100755
index 0000000..cfbb359
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/ReachibilityRealComplexNoConnectivity/part-1
@@ -0,0 +1,7 @@
+1	1
+5	1
+9	1
+13	1
+17	1
+21	0
+25	2
diff --git a/pregelix/pregelix-example/src/test/resources/expected/ReachibilityRealComplexNoConnectivity/part-2 b/pregelix/pregelix-example/src/test/resources/expected/ReachibilityRealComplexNoConnectivity/part-2
new file mode 100755
index 0000000..8e3ca7c
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/ReachibilityRealComplexNoConnectivity/part-2
@@ -0,0 +1,5 @@
+2	1
+6	1
+10	1
+14	1
+18	1
diff --git a/pregelix/pregelix-example/src/test/resources/expected/ReachibilityRealComplexNoConnectivity/part-3 b/pregelix/pregelix-example/src/test/resources/expected/ReachibilityRealComplexNoConnectivity/part-3
new file mode 100755
index 0000000..5f83a3d
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/ReachibilityRealComplexNoConnectivity/part-3
@@ -0,0 +1,6 @@
+3	1
+7	1
+11	1
+15	1
+19	1
+27	0
diff --git a/pregelix/pregelix-example/src/test/resources/expected/ShortestPaths.result b/pregelix/pregelix-example/src/test/resources/expected/ShortestPaths.result
deleted file mode 100644
index 46d1c73..0000000
--- a/pregelix/pregelix-example/src/test/resources/expected/ShortestPaths.result
+++ /dev/null
@@ -1,20 +0,0 @@
-0 0.0
-1 0.0
-2 100.0
-3 300.0
-4 600.0
-5 1000.0
-6 1500.0
-7 2100.0
-8 2800.0
-9 3600.0
-10 4500.0
-11 5500.0
-12 6600.0
-13 7800.0
-14 9100.0
-15 10500.0
-16 12000.0
-17 13600.0
-18 15300.0
-19 17100.0
diff --git a/pregelix/pregelix-example/src/test/resources/expected/ShortestPaths/part-0 b/pregelix/pregelix-example/src/test/resources/expected/ShortestPaths/part-0
new file mode 100755
index 0000000..e8eadcd
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/ShortestPaths/part-0
@@ -0,0 +1,10 @@
+0	0.0
+4	600.0
+8	2800.0
+12	6600.0
+16	12000.0
+20	19000.0
+24	27600.0
+28	37800.0
+32	49600.0
+36	63000.0
diff --git a/pregelix/pregelix-example/src/test/resources/expected/ShortestPaths/part-1 b/pregelix/pregelix-example/src/test/resources/expected/ShortestPaths/part-1
new file mode 100755
index 0000000..c3d200b
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/ShortestPaths/part-1
@@ -0,0 +1,10 @@
+1	0.0
+5	1000.0
+9	3600.0
+13	7800.0
+17	13600.0
+21	21000.0
+25	30000.0
+29	40600.0
+33	52800.0
+37	66600.0
diff --git a/pregelix/pregelix-example/src/test/resources/expected/ShortestPaths/part-2 b/pregelix/pregelix-example/src/test/resources/expected/ShortestPaths/part-2
new file mode 100755
index 0000000..779a683
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/ShortestPaths/part-2
@@ -0,0 +1,10 @@
+2	100.0
+6	1500.0
+10	4500.0
+14	9100.0
+18	15300.0
+22	23100.0
+26	32500.0
+30	43500.0
+34	56100.0
+38	70300.0
diff --git a/pregelix/pregelix-example/src/test/resources/expected/ShortestPaths/part-3 b/pregelix/pregelix-example/src/test/resources/expected/ShortestPaths/part-3
new file mode 100755
index 0000000..57d26c9
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/ShortestPaths/part-3
@@ -0,0 +1,10 @@
+3	300.0
+7	2100.0
+11	5500.0
+15	10500.0
+19	17100.0
+23	25300.0
+27	35100.0
+31	46500.0
+35	59500.0
+39	74100.0
diff --git a/pregelix/pregelix-example/src/test/resources/expected/ShortestPathsReal.result b/pregelix/pregelix-example/src/test/resources/expected/ShortestPathsReal.result
deleted file mode 100644
index b42462f..0000000
--- a/pregelix/pregelix-example/src/test/resources/expected/ShortestPathsReal.result
+++ /dev/null
@@ -1,20 +0,0 @@
-0 0.0
-1 1.0
-2 2.0
-3 3.0
-4 4.0
-5 5.0
-6 6.0
-7 7.0
-8 8.0
-9 9.0
-10 10.0
-11 11.0
-12 12.0
-13 13.0
-14 14.0
-15 15.0
-16 16.0
-17 17.0
-18 18.0
-19 19.0
diff --git a/pregelix/pregelix-example/src/test/resources/expected/ShortestPathsReal/part-0 b/pregelix/pregelix-example/src/test/resources/expected/ShortestPathsReal/part-0
new file mode 100755
index 0000000..a1cabe2
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/ShortestPathsReal/part-0
@@ -0,0 +1,5 @@
+0	0.0
+4	4.0
+8	8.0
+12	12.0
+16	16.0
diff --git a/pregelix/pregelix-example/src/test/resources/expected/ShortestPathsReal/part-1 b/pregelix/pregelix-example/src/test/resources/expected/ShortestPathsReal/part-1
new file mode 100755
index 0000000..303ed2a
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/ShortestPathsReal/part-1
@@ -0,0 +1,5 @@
+1	1.0
+5	5.0
+9	9.0
+13	13.0
+17	17.0
diff --git a/pregelix/pregelix-example/src/test/resources/expected/ShortestPathsReal/part-2 b/pregelix/pregelix-example/src/test/resources/expected/ShortestPathsReal/part-2
new file mode 100755
index 0000000..0020d25
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/ShortestPathsReal/part-2
@@ -0,0 +1,5 @@
+2	2.0
+6	6.0
+10	10.0
+14	14.0
+18	18.0
diff --git a/pregelix/pregelix-example/src/test/resources/expected/ShortestPathsReal/part-3 b/pregelix/pregelix-example/src/test/resources/expected/ShortestPathsReal/part-3
new file mode 100755
index 0000000..e2657bd
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/ShortestPathsReal/part-3
@@ -0,0 +1,5 @@
+3	3.0
+7	7.0
+11	11.0
+15	15.0
+19	19.0
diff --git a/pregelix/pregelix-example/src/test/resources/expected/TriangleCounting.result b/pregelix/pregelix-example/src/test/resources/expected/TriangleCounting.result
deleted file mode 100644
index 4818e13..0000000
--- a/pregelix/pregelix-example/src/test/resources/expected/TriangleCounting.result
+++ /dev/null
@@ -1,7 +0,0 @@
-1 3
-2 2
-3 0
-4 0
-5 1
-6 0
-7 0
diff --git a/pregelix/pregelix-example/src/test/resources/expected/TriangleCounting/.part-0.crc b/pregelix/pregelix-example/src/test/resources/expected/TriangleCounting/.part-0.crc
new file mode 100644
index 0000000..61e7df3
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/TriangleCounting/.part-0.crc
Binary files differ
diff --git a/pregelix/pregelix-example/src/test/resources/expected/TriangleCounting/.part-1.crc b/pregelix/pregelix-example/src/test/resources/expected/TriangleCounting/.part-1.crc
new file mode 100644
index 0000000..9f34827
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/TriangleCounting/.part-1.crc
Binary files differ
diff --git a/pregelix/pregelix-example/src/test/resources/expected/TriangleCounting/.part-2.crc b/pregelix/pregelix-example/src/test/resources/expected/TriangleCounting/.part-2.crc
new file mode 100644
index 0000000..cfe7fa2
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/TriangleCounting/.part-2.crc
Binary files differ
diff --git a/pregelix/pregelix-example/src/test/resources/expected/TriangleCounting/.part-3.crc b/pregelix/pregelix-example/src/test/resources/expected/TriangleCounting/.part-3.crc
new file mode 100644
index 0000000..9959654
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/TriangleCounting/.part-3.crc
Binary files differ
diff --git a/pregelix/pregelix-example/src/test/resources/expected/TriangleCounting/part-0 b/pregelix/pregelix-example/src/test/resources/expected/TriangleCounting/part-0
new file mode 100755
index 0000000..b36d2b5
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/TriangleCounting/part-0
@@ -0,0 +1 @@
+4	0
diff --git a/pregelix/pregelix-example/src/test/resources/expected/TriangleCounting/part-1 b/pregelix/pregelix-example/src/test/resources/expected/TriangleCounting/part-1
new file mode 100755
index 0000000..2812c76
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/TriangleCounting/part-1
@@ -0,0 +1,2 @@
+1	3
+5	1
diff --git a/pregelix/pregelix-example/src/test/resources/expected/TriangleCounting/part-2 b/pregelix/pregelix-example/src/test/resources/expected/TriangleCounting/part-2
new file mode 100755
index 0000000..25cfb7b
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/TriangleCounting/part-2
@@ -0,0 +1,2 @@
+2	2
+6	0
diff --git a/pregelix/pregelix-example/src/test/resources/expected/TriangleCounting/part-3 b/pregelix/pregelix-example/src/test/resources/expected/TriangleCounting/part-3
new file mode 100755
index 0000000..2bd8d38
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/expected/TriangleCounting/part-3
@@ -0,0 +1,2 @@
+3	0
+7	0
diff --git a/pregelix/pregelix-example/src/test/resources/hyracks-deployment.properties b/pregelix/pregelix-example/src/test/resources/hyracks-deployment.properties
deleted file mode 100644
index 9c42b89..0000000
--- a/pregelix/pregelix-example/src/test/resources/hyracks-deployment.properties
+++ /dev/null
@@ -1,2 +0,0 @@
-#cc.bootstrap.class=edu.uci.ics.asterix.hyracks.bootstrap.CCBootstrapImpl
-nc.bootstrap.class=edu.uci.ics.pregelix.runtime.bootstrap.NCBootstrapImpl
diff --git a/pregelix/pregelix-example/src/test/resources/jobs/MaximalClique2.xml b/pregelix/pregelix-example/src/test/resources/jobs/MaximalClique2.xml
new file mode 100644
index 0000000..5621259
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/jobs/MaximalClique2.xml
@@ -0,0 +1,142 @@
+<?xml version="1.0" encoding="UTF-8" standalone="no"?><configuration>
+<property><name>mapred.tasktracker.dns.nameserver</name><value>default</value></property>
+<property><name>mapred.queue.default.acl-administer-jobs</name><value>*</value></property>
+<property><name>mapred.skip.map.auto.incr.proc.count</name><value>true</value></property>
+<property><name>mapred.jobtracker.instrumentation</name><value>org.apache.hadoop.mapred.JobTrackerMetricsInst</value></property>
+<property><name>mapred.skip.reduce.auto.incr.proc.count</name><value>true</value></property>
+<property><name>fs.hsftp.impl</name><value>org.apache.hadoop.hdfs.HsftpFileSystem</value></property>
+<property><name>mapred.input.dir</name><value>file:/clique2</value></property>
+<property><name>mapred.submit.replication</name><value>10</value></property>
+<property><name>ipc.server.tcpnodelay</name><value>false</value></property>
+<property><name>fs.checkpoint.dir</name><value>${hadoop.tmp.dir}/dfs/namesecondary</value></property>
+<property><name>mapred.output.compression.type</name><value>RECORD</value></property>
+<property><name>mapred.job.shuffle.merge.percent</name><value>0.66</value></property>
+<property><name>mapred.child.java.opts</name><value>-Xmx200m</value></property>
+<property><name>mapred.queue.default.acl-submit-job</name><value>*</value></property>
+<property><name>keep.failed.task.files</name><value>false</value></property>
+<property><name>mapred.jobtracker.job.history.block.size</name><value>3145728</value></property>
+<property><name>io.bytes.per.checksum</name><value>512</value></property>
+<property><name>mapred.task.tracker.report.address</name><value>127.0.0.1:0</value></property>
+<property><name>hadoop.util.hash.type</name><value>murmur</value></property>
+<property><name>fs.hdfs.impl</name><value>org.apache.hadoop.hdfs.DistributedFileSystem</value></property>
+<property><name>fs.ramfs.impl</name><value>org.apache.hadoop.fs.InMemoryFileSystem</value></property>
+<property><name>mapred.jobtracker.restart.recover</name><value>false</value></property>
+<property><name>fs.hftp.impl</name><value>org.apache.hadoop.hdfs.HftpFileSystem</value></property>
+<property><name>fs.checkpoint.period</name><value>3600</value></property>
+<property><name>mapred.child.tmp</name><value>./tmp</value></property>
+<property><name>mapred.local.dir.minspacekill</name><value>0</value></property>
+<property><name>map.sort.class</name><value>org.apache.hadoop.util.QuickSort</value></property>
+<property><name>hadoop.logfile.count</name><value>10</value></property>
+<property><name>ipc.client.connection.maxidletime</name><value>10000</value></property>
+<property><name>mapred.output.dir</name><value>/resultclique</value></property>
+<property><name>io.map.index.skip</name><value>0</value></property>
+<property><name>mapred.tasktracker.expiry.interval</name><value>600000</value></property>
+<property><name>mapred.output.compress</name><value>false</value></property>
+<property><name>io.seqfile.lazydecompress</name><value>true</value></property>
+<property><name>mapred.reduce.parallel.copies</name><value>5</value></property>
+<property><name>fs.checkpoint.size</name><value>67108864</value></property>
+<property><name>mapred.job.reduce.input.buffer.percent</name><value>0.0</value></property>
+<property><name>mapred.job.name</name><value>Maximal Clique 2</value></property>
+<property><name>local.cache.size</name><value>10737418240</value></property>
+<property><name>fs.s3n.impl</name><value>org.apache.hadoop.fs.s3native.NativeS3FileSystem</value></property>
+<property><name>mapred.userlog.limit.kb</name><value>0</value></property>
+<property><name>fs.file.impl</name><value>org.apache.hadoop.fs.LocalFileSystem</value></property>
+<property><name>mapred.task.tracker.http.address</name><value>0.0.0.0:50060</value></property>
+<property><name>mapred.task.timeout</name><value>600000</value></property>
+<property><name>fs.kfs.impl</name><value>org.apache.hadoop.fs.kfs.KosmosFileSystem</value></property>
+<property><name>mapred.max.tracker.blacklists</name><value>4</value></property>
+<property><name>fs.s3.buffer.dir</name><value>${hadoop.tmp.dir}/s3</value></property>
+<property><name>mapred.job.tracker.persist.jobstatus.dir</name><value>/jobtracker/jobsInfo</value></property>
+<property><name>ipc.client.kill.max</name><value>10</value></property>
+<property><name>mapred.tasktracker.instrumentation</name><value>org.apache.hadoop.mapred.TaskTrackerMetricsInst</value></property>
+<property><name>mapred.reduce.tasks.speculative.execution</name><value>true</value></property>
+<property><name>io.sort.record.percent</name><value>0.05</value></property>
+<property><name>hadoop.security.authorization</name><value>false</value></property>
+<property><name>mapred.max.tracker.failures</name><value>4</value></property>
+<property><name>mapred.jobtracker.taskScheduler</name><value>org.apache.hadoop.mapred.JobQueueTaskScheduler</value></property>
+<property><name>mapred.tasktracker.dns.interface</name><value>default</value></property>
+<property><name>mapred.map.tasks</name><value>2</value></property>
+<property><name>mapred.job.tracker.persist.jobstatus.hours</name><value>0</value></property>
+<property><name>fs.s3.sleepTimeSeconds</name><value>10</value></property>
+<property><name>fs.default.name</name><value>file:///</value></property>
+<property><name>tasktracker.http.threads</name><value>40</value></property>
+<property><name>mapred.tasktracker.taskmemorymanager.monitoring-interval</name><value>5000</value></property>
+<property><name>hadoop.rpc.socket.factory.class.default</name><value>org.apache.hadoop.net.StandardSocketFactory</value></property>
+<property><name>mapred.reduce.tasks</name><value>1</value></property>
+<property><name>topology.node.switch.mapping.impl</name><value>org.apache.hadoop.net.ScriptBasedMapping</value></property>
+<property><name>pregelix.vertexClass</name><value>edu.uci.ics.pregelix.example.maximalclique.MaximalCliqueVertex</value></property>
+<property><name>mapred.skip.reduce.max.skip.groups</name><value>0</value></property>
+<property><name>io.file.buffer.size</name><value>4096</value></property>
+<property><name>mapred.jobtracker.maxtasks.per.job</name><value>-1</value></property>
+<property><name>mapred.tasktracker.indexcache.mb</name><value>10</value></property>
+<property><name>mapred.tasktracker.map.tasks.maximum</name><value>2</value></property>
+<property><name>fs.har.impl.disable.cache</name><value>true</value></property>
+<property><name>mapred.task.profile.maps</name><value>0-2</value></property>
+<property><name>hadoop.native.lib</name><value>true</value></property>
+<property><name>fs.s3.block.size</name><value>67108864</value></property>
+<property><name>mapred.job.reuse.jvm.num.tasks</name><value>1</value></property>
+<property><name>mapred.job.tracker.http.address</name><value>0.0.0.0:50030</value></property>
+<property><name>mapred.tasktracker.reduce.tasks.maximum</name><value>2</value></property>
+<property><name>io.compression.codecs</name><value>org.apache.hadoop.io.compress.DefaultCodec,org.apache.hadoop.io.compress.GzipCodec,org.apache.hadoop.io.compress.BZip2Codec</value></property>
+<property><name>mapred.job.shuffle.input.buffer.percent</name><value>0.70</value></property>
+<property><name>io.seqfile.compress.blocksize</name><value>1000000</value></property>
+<property><name>mapred.queue.names</name><value>default</value></property>
+<property><name>fs.har.impl</name><value>org.apache.hadoop.fs.HarFileSystem</value></property>
+<property><name>io.mapfile.bloom.error.rate</name><value>0.005</value></property>
+<property><name>mapred.job.tracker</name><value>local</value></property>
+<property><name>io.skip.checksum.errors</name><value>false</value></property>
+<property><name>mapred.reduce.max.attempts</name><value>4</value></property>
+<property><name>fs.s3.maxRetries</name><value>4</value></property>
+<property><name>ipc.server.listen.queue.size</name><value>128</value></property>
+<property><name>fs.trash.interval</name><value>0</value></property>
+<property><name>mapred.local.dir.minspacestart</name><value>0</value></property>
+<property><name>fs.s3.impl</name><value>org.apache.hadoop.fs.s3.S3FileSystem</value></property>
+<property><name>io.seqfile.sorter.recordlimit</name><value>1000000</value></property>
+<property><name>io.mapfile.bloom.size</name><value>1048576</value></property>
+<property><name>io.sort.mb</name><value>100</value></property>
+<property><name>mapred.local.dir</name><value>${hadoop.tmp.dir}/mapred/local</value></property>
+<property><name>io.sort.factor</name><value>10</value></property>
+<property><name>mapred.task.profile</name><value>false</value></property>
+<property><name>job.end.retry.interval</name><value>30000</value></property>
+<property><name>mapred.tasktracker.procfsbasedprocesstree.sleeptime-before-sigkill</name><value>5000</value></property>
+<property><name>mapred.jobtracker.completeuserjobs.maximum</name><value>100</value></property>
+<property><name>mapred.task.profile.reduces</name><value>0-2</value></property>
+<property><name>webinterface.private.actions</name><value>false</value></property>
+<property><name>hadoop.tmp.dir</name><value>/tmp/hadoop-${user.name}</value></property>
+<property><name>mapred.output.compression.codec</name><value>org.apache.hadoop.io.compress.DefaultCodec</value></property>
+<property><name>mapred.skip.attempts.to.start.skipping</name><value>2</value></property>
+<property><name>mapred.temp.dir</name><value>${hadoop.tmp.dir}/mapred/temp</value></property>
+<property><name>mapred.merge.recordsBeforeProgress</name><value>10000</value></property>
+<property><name>mapred.map.output.compression.codec</name><value>org.apache.hadoop.io.compress.DefaultCodec</value></property>
+<property><name>mapred.compress.map.output</name><value>false</value></property>
+<property><name>io.sort.spill.percent</name><value>0.80</value></property>
+<property><name>fs.checkpoint.edits.dir</name><value>${fs.checkpoint.dir}</value></property>
+<property><name>mapred.userlog.retain.hours</name><value>24</value></property>
+<property><name>mapred.system.dir</name><value>${hadoop.tmp.dir}/mapred/system</value></property>
+<property><name>mapred.line.input.format.linespermap</name><value>1</value></property>
+<property><name>job.end.retry.attempts</name><value>0</value></property>
+<property><name>ipc.client.idlethreshold</name><value>4000</value></property>
+<property><name>pregelix.vertexOutputFormatClass</name><value>edu.uci.ics.pregelix.example.maximalclique.MaximalCliqueVertex$MaximalCliqueVertexOutputFormat</value></property>
+<property><name>mapred.reduce.copy.backoff</name><value>300</value></property>
+<property><name>mapred.map.tasks.speculative.execution</name><value>true</value></property>
+<property><name>mapred.inmem.merge.threshold</name><value>1000</value></property>
+<property><name>hadoop.logfile.size</name><value>10000000</value></property>
+<property><name>pregelix.vertexInputFormatClass</name><value>edu.uci.ics.pregelix.example.maximalclique.TextMaximalCliqueInputFormat</value></property>
+<property><name>pregelix.aggregatorClass</name><value>edu.uci.ics.pregelix.example.maximalclique.MaximalCliqueAggregator</value></property>
+<property><name>mapred.job.queue.name</name><value>default</value></property>
+<property><name>mapred.job.tracker.persist.jobstatus.active</name><value>false</value></property>
+<property><name>pregelix.incStateLength</name><value>true</value></property>
+<property><name>mapred.reduce.slowstart.completed.maps</name><value>0.05</value></property>
+<property><name>topology.script.number.args</name><value>100</value></property>
+<property><name>mapred.skip.map.max.skip.records</name><value>0</value></property>
+<property><name>fs.ftp.impl</name><value>org.apache.hadoop.fs.ftp.FTPFileSystem</value></property>
+<property><name>mapred.task.cache.levels</name><value>2</value></property>
+<property><name>mapred.job.tracker.handler.count</name><value>10</value></property>
+<property><name>io.serializations</name><value>org.apache.hadoop.io.serializer.WritableSerialization</value></property>
+<property><name>ipc.client.connect.max.retries</name><value>10</value></property>
+<property><name>mapred.min.split.size</name><value>0</value></property>
+<property><name>mapred.map.max.attempts</name><value>4</value></property>
+<property><name>jobclient.output.filter</name><value>FAILED</value></property>
+<property><name>ipc.client.tcpnodelay</name><value>false</value></property>
+<property><name>mapred.acls.enabled</name><value>false</value></property>
+</configuration>
\ No newline at end of file
diff --git a/pregelix/pregelix-example/src/test/resources/jobs/MaximalClique3.xml b/pregelix/pregelix-example/src/test/resources/jobs/MaximalClique3.xml
new file mode 100644
index 0000000..d4f81ba
--- /dev/null
+++ b/pregelix/pregelix-example/src/test/resources/jobs/MaximalClique3.xml
@@ -0,0 +1,142 @@
+<?xml version="1.0" encoding="UTF-8" standalone="no"?><configuration>
+<property><name>mapred.tasktracker.dns.nameserver</name><value>default</value></property>
+<property><name>mapred.queue.default.acl-administer-jobs</name><value>*</value></property>
+<property><name>mapred.skip.map.auto.incr.proc.count</name><value>true</value></property>
+<property><name>mapred.jobtracker.instrumentation</name><value>org.apache.hadoop.mapred.JobTrackerMetricsInst</value></property>
+<property><name>mapred.skip.reduce.auto.incr.proc.count</name><value>true</value></property>
+<property><name>fs.hsftp.impl</name><value>org.apache.hadoop.hdfs.HsftpFileSystem</value></property>
+<property><name>mapred.input.dir</name><value>file:/clique3</value></property>
+<property><name>mapred.submit.replication</name><value>10</value></property>
+<property><name>ipc.server.tcpnodelay</name><value>false</value></property>
+<property><name>fs.checkpoint.dir</name><value>${hadoop.tmp.dir}/dfs/namesecondary</value></property>
+<property><name>mapred.output.compression.type</name><value>RECORD</value></property>
+<property><name>mapred.job.shuffle.merge.percent</name><value>0.66</value></property>
+<property><name>mapred.child.java.opts</name><value>-Xmx200m</value></property>
+<property><name>mapred.queue.default.acl-submit-job</name><value>*</value></property>
+<property><name>keep.failed.task.files</name><value>false</value></property>
+<property><name>mapred.jobtracker.job.history.block.size</name><value>3145728</value></property>
+<property><name>io.bytes.per.checksum</name><value>512</value></property>
+<property><name>mapred.task.tracker.report.address</name><value>127.0.0.1:0</value></property>
+<property><name>hadoop.util.hash.type</name><value>murmur</value></property>
+<property><name>fs.hdfs.impl</name><value>org.apache.hadoop.hdfs.DistributedFileSystem</value></property>
+<property><name>fs.ramfs.impl</name><value>org.apache.hadoop.fs.InMemoryFileSystem</value></property>
+<property><name>mapred.jobtracker.restart.recover</name><value>false</value></property>
+<property><name>fs.hftp.impl</name><value>org.apache.hadoop.hdfs.HftpFileSystem</value></property>
+<property><name>fs.checkpoint.period</name><value>3600</value></property>
+<property><name>mapred.child.tmp</name><value>./tmp</value></property>
+<property><name>mapred.local.dir.minspacekill</name><value>0</value></property>
+<property><name>map.sort.class</name><value>org.apache.hadoop.util.QuickSort</value></property>
+<property><name>hadoop.logfile.count</name><value>10</value></property>
+<property><name>ipc.client.connection.maxidletime</name><value>10000</value></property>
+<property><name>mapred.output.dir</name><value>/resultclique</value></property>
+<property><name>io.map.index.skip</name><value>0</value></property>
+<property><name>mapred.tasktracker.expiry.interval</name><value>600000</value></property>
+<property><name>mapred.output.compress</name><value>false</value></property>
+<property><name>io.seqfile.lazydecompress</name><value>true</value></property>
+<property><name>mapred.reduce.parallel.copies</name><value>5</value></property>
+<property><name>fs.checkpoint.size</name><value>67108864</value></property>
+<property><name>mapred.job.reduce.input.buffer.percent</name><value>0.0</value></property>
+<property><name>mapred.job.name</name><value>Maximal Clique 3</value></property>
+<property><name>local.cache.size</name><value>10737418240</value></property>
+<property><name>fs.s3n.impl</name><value>org.apache.hadoop.fs.s3native.NativeS3FileSystem</value></property>
+<property><name>mapred.userlog.limit.kb</name><value>0</value></property>
+<property><name>fs.file.impl</name><value>org.apache.hadoop.fs.LocalFileSystem</value></property>
+<property><name>mapred.task.tracker.http.address</name><value>0.0.0.0:50060</value></property>
+<property><name>mapred.task.timeout</name><value>600000</value></property>
+<property><name>fs.kfs.impl</name><value>org.apache.hadoop.fs.kfs.KosmosFileSystem</value></property>
+<property><name>mapred.max.tracker.blacklists</name><value>4</value></property>
+<property><name>fs.s3.buffer.dir</name><value>${hadoop.tmp.dir}/s3</value></property>
+<property><name>mapred.job.tracker.persist.jobstatus.dir</name><value>/jobtracker/jobsInfo</value></property>
+<property><name>ipc.client.kill.max</name><value>10</value></property>
+<property><name>mapred.tasktracker.instrumentation</name><value>org.apache.hadoop.mapred.TaskTrackerMetricsInst</value></property>
+<property><name>mapred.reduce.tasks.speculative.execution</name><value>true</value></property>
+<property><name>io.sort.record.percent</name><value>0.05</value></property>
+<property><name>hadoop.security.authorization</name><value>false</value></property>
+<property><name>mapred.max.tracker.failures</name><value>4</value></property>
+<property><name>mapred.jobtracker.taskScheduler</name><value>org.apache.hadoop.mapred.JobQueueTaskScheduler</value></property>
+<property><name>mapred.tasktracker.dns.interface</name><value>default</value></property>
+<property><name>mapred.map.tasks</name><value>2</value></property>
+<property><name>mapred.job.tracker.persist.jobstatus.hours</name><value>0</value></property>
+<property><name>fs.s3.sleepTimeSeconds</name><value>10</value></property>
+<property><name>fs.default.name</name><value>file:///</value></property>
+<property><name>tasktracker.http.threads</name><value>40</value></property>
+<property><name>mapred.tasktracker.taskmemorymanager.monitoring-interval</name><value>5000</value></property>
+<property><name>hadoop.rpc.socket.factory.class.default</name><value>org.apache.hadoop.net.StandardSocketFactory</value></property>
+<property><name>mapred.reduce.tasks</name><value>1</value></property>
+<property><name>topology.node.switch.mapping.impl</name><value>org.apache.hadoop.net.ScriptBasedMapping</value></property>
+<property><name>pregelix.vertexClass</name><value>edu.uci.ics.pregelix.example.maximalclique.MaximalCliqueVertex</value></property>
+<property><name>mapred.skip.reduce.max.skip.groups</name><value>0</value></property>
+<property><name>io.file.buffer.size</name><value>4096</value></property>
+<property><name>mapred.jobtracker.maxtasks.per.job</name><value>-1</value></property>
+<property><name>mapred.tasktracker.indexcache.mb</name><value>10</value></property>
+<property><name>mapred.tasktracker.map.tasks.maximum</name><value>2</value></property>
+<property><name>fs.har.impl.disable.cache</name><value>true</value></property>
+<property><name>mapred.task.profile.maps</name><value>0-2</value></property>
+<property><name>hadoop.native.lib</name><value>true</value></property>
+<property><name>fs.s3.block.size</name><value>67108864</value></property>
+<property><name>mapred.job.reuse.jvm.num.tasks</name><value>1</value></property>
+<property><name>mapred.job.tracker.http.address</name><value>0.0.0.0:50030</value></property>
+<property><name>mapred.tasktracker.reduce.tasks.maximum</name><value>2</value></property>
+<property><name>io.compression.codecs</name><value>org.apache.hadoop.io.compress.DefaultCodec,org.apache.hadoop.io.compress.GzipCodec,org.apache.hadoop.io.compress.BZip2Codec</value></property>
+<property><name>mapred.job.shuffle.input.buffer.percent</name><value>0.70</value></property>
+<property><name>io.seqfile.compress.blocksize</name><value>1000000</value></property>
+<property><name>mapred.queue.names</name><value>default</value></property>
+<property><name>fs.har.impl</name><value>org.apache.hadoop.fs.HarFileSystem</value></property>
+<property><name>io.mapfile.bloom.error.rate</name><value>0.005</value></property>
+<property><name>mapred.job.tracker</name><value>local</value></property>
+<property><name>io.skip.checksum.errors</name><value>false</value></property>
+<property><name>mapred.reduce.max.attempts</name><value>4</value></property>
+<property><name>fs.s3.maxRetries</name><value>4</value></property>
+<property><name>ipc.server.listen.queue.size</name><value>128</value></property>
+<property><name>fs.trash.interval</name><value>0</value></property>
+<property><name>mapred.local.dir.minspacestart</name><value>0</value></property>
+<property><name>fs.s3.impl</name><value>org.apache.hadoop.fs.s3.S3FileSystem</value></property>
+<property><name>io.seqfile.sorter.recordlimit</name><value>1000000</value></property>
+<property><name>io.mapfile.bloom.size</name><value>1048576</value></property>
+<property><name>io.sort.mb</name><value>100</value></property>
+<property><name>mapred.local.dir</name><value>${hadoop.tmp.dir}/mapred/local</value></property>
+<property><name>io.sort.factor</name><value>10</value></property>
+<property><name>mapred.task.profile</name><value>false</value></property>
+<property><name>job.end.retry.interval</name><value>30000</value></property>
+<property><name>mapred.tasktracker.procfsbasedprocesstree.sleeptime-before-sigkill</name><value>5000</value></property>
+<property><name>mapred.jobtracker.completeuserjobs.maximum</name><value>100</value></property>
+<property><name>mapred.task.profile.reduces</name><value>0-2</value></property>
+<property><name>webinterface.private.actions</name><value>false</value></property>
+<property><name>hadoop.tmp.dir</name><value>/tmp/hadoop-${user.name}</value></property>
+<property><name>mapred.output.compression.codec</name><value>org.apache.hadoop.io.compress.DefaultCodec</value></property>
+<property><name>mapred.skip.attempts.to.start.skipping</name><value>2</value></property>
+<property><name>mapred.temp.dir</name><value>${hadoop.tmp.dir}/mapred/temp</value></property>
+<property><name>mapred.merge.recordsBeforeProgress</name><value>10000</value></property>
+<property><name>mapred.map.output.compression.codec</name><value>org.apache.hadoop.io.compress.DefaultCodec</value></property>
+<property><name>mapred.compress.map.output</name><value>false</value></property>
+<property><name>io.sort.spill.percent</name><value>0.80</value></property>
+<property><name>fs.checkpoint.edits.dir</name><value>${fs.checkpoint.dir}</value></property>
+<property><name>mapred.userlog.retain.hours</name><value>24</value></property>
+<property><name>mapred.system.dir</name><value>${hadoop.tmp.dir}/mapred/system</value></property>
+<property><name>mapred.line.input.format.linespermap</name><value>1</value></property>
+<property><name>job.end.retry.attempts</name><value>0</value></property>
+<property><name>ipc.client.idlethreshold</name><value>4000</value></property>
+<property><name>pregelix.vertexOutputFormatClass</name><value>edu.uci.ics.pregelix.example.maximalclique.MaximalCliqueVertex$MaximalCliqueVertexOutputFormat</value></property>
+<property><name>mapred.reduce.copy.backoff</name><value>300</value></property>
+<property><name>mapred.map.tasks.speculative.execution</name><value>true</value></property>
+<property><name>mapred.inmem.merge.threshold</name><value>1000</value></property>
+<property><name>hadoop.logfile.size</name><value>10000000</value></property>
+<property><name>pregelix.vertexInputFormatClass</name><value>edu.uci.ics.pregelix.example.maximalclique.TextMaximalCliqueInputFormat</value></property>
+<property><name>pregelix.aggregatorClass</name><value>edu.uci.ics.pregelix.example.maximalclique.MaximalCliqueAggregator</value></property>
+<property><name>mapred.job.queue.name</name><value>default</value></property>
+<property><name>mapred.job.tracker.persist.jobstatus.active</name><value>false</value></property>
+<property><name>pregelix.incStateLength</name><value>true</value></property>
+<property><name>mapred.reduce.slowstart.completed.maps</name><value>0.05</value></property>
+<property><name>topology.script.number.args</name><value>100</value></property>
+<property><name>mapred.skip.map.max.skip.records</name><value>0</value></property>
+<property><name>fs.ftp.impl</name><value>org.apache.hadoop.fs.ftp.FTPFileSystem</value></property>
+<property><name>mapred.task.cache.levels</name><value>2</value></property>
+<property><name>mapred.job.tracker.handler.count</name><value>10</value></property>
+<property><name>io.serializations</name><value>org.apache.hadoop.io.serializer.WritableSerialization</value></property>
+<property><name>ipc.client.connect.max.retries</name><value>10</value></property>
+<property><name>mapred.min.split.size</name><value>0</value></property>
+<property><name>mapred.map.max.attempts</name><value>4</value></property>
+<property><name>jobclient.output.filter</name><value>FAILED</value></property>
+<property><name>ipc.client.tcpnodelay</name><value>false</value></property>
+<property><name>mapred.acls.enabled</name><value>false</value></property>
+</configuration>
\ No newline at end of file