clean up examples
git-svn-id: https://hyracks.googlecode.com/svn/branches/fullstack_staging@2695 123451ca-8445-de46-9d55-352943316053
diff --git a/pregelix/pregelix-example/src/main/java/edu/uci/ics/pregelix/example/PageRankVertex.java b/pregelix/pregelix-example/src/main/java/edu/uci/ics/pregelix/example/PageRankVertex.java
index 586e1f2..02e1625 100644
--- a/pregelix/pregelix-example/src/main/java/edu/uci/ics/pregelix/example/PageRankVertex.java
+++ b/pregelix/pregelix-example/src/main/java/edu/uci/ics/pregelix/example/PageRankVertex.java
@@ -123,13 +123,13 @@
/**
* Simple VertexReader that supports {@link SimplePageRankVertex}
*/
- public static class SimplePageRankVertexReader extends
+ public static class SimulatedPageRankVertexReader extends
GeneratedVertexReader<VLongWritable, DoubleWritable, FloatWritable, DoubleWritable> {
/** Class logger */
- private static final Logger LOG = Logger.getLogger(SimplePageRankVertexReader.class.getName());
+ private static final Logger LOG = Logger.getLogger(SimulatedPageRankVertexReader.class.getName());
private Map<VLongWritable, FloatWritable> edges = Maps.newHashMap();
- public SimplePageRankVertexReader() {
+ public SimulatedPageRankVertexReader() {
super();
}
@@ -162,12 +162,12 @@
/**
* Simple VertexInputFormat that supports {@link SimplePageRankVertex}
*/
- public static class SimplePageRankVertexInputFormat extends
+ public static class SimulatedPageRankVertexInputFormat extends
GeneratedVertexInputFormat<VLongWritable, DoubleWritable, FloatWritable, DoubleWritable> {
@Override
public VertexReader<VLongWritable, DoubleWritable, FloatWritable, DoubleWritable> createVertexReader(
InputSplit split, TaskAttemptContext context) throws IOException {
- return new SimplePageRankVertexReader();
+ return new SimulatedPageRankVertexReader();
}
}
@@ -187,7 +187,7 @@
new Text(vertex.getVertexValue().toString()));
}
}
-
+
@Override
public String toString() {
return getVertexId() + " " + getVertexValue();
diff --git a/pregelix/pregelix-example/src/main/java/edu/uci/ics/pregelix/example/maximalclique/AdjacencyListWritable.java b/pregelix/pregelix-example/src/main/java/edu/uci/ics/pregelix/example/maximalclique/AdjacencyListWritable.java
index b07017a..83e0a6b 100644
--- a/pregelix/pregelix-example/src/main/java/edu/uci/ics/pregelix/example/maximalclique/AdjacencyListWritable.java
+++ b/pregelix/pregelix-example/src/main/java/edu/uci/ics/pregelix/example/maximalclique/AdjacencyListWritable.java
@@ -28,7 +28,7 @@
import edu.uci.ics.pregelix.example.io.VLongWritable;
/**
- * The adjacency list is the value part of the mapper output
+ * The adjacency list contains <src, list-of-neighbors>
*/
public class AdjacencyListWritable implements Writable {
@@ -72,15 +72,6 @@
}
}
- public AdjacencyListWritable clone() {
- AdjacencyListWritable clone = new AdjacencyListWritable();
- clone.setSource(sourceVertex);
- for (VLongWritable dest : destinationVertexes) {
- clone.addNeighbor(dest);
- }
- return clone;
- }
-
public int numberOfNeighbors() {
return destinationVertexes.size();
}
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 940fe2b..69cb336 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
@@ -25,6 +25,9 @@
import edu.uci.ics.pregelix.example.io.VLongWritable;
+/**
+ * The representation of cliques stored in a vertex.
+ */
public class CliquesWritable implements Writable {
private List<VLongWritable> cliques = new ArrayList<VLongWritable>();
@@ -39,18 +42,45 @@
}
- public void setClusterSize(int sizeOfClique) {
+ /**
+ * Set the size of cliques.
+ *
+ * @param sizeOfClique
+ * the size of each maximal clique
+ */
+ public void setCliqueSize(int sizeOfClique) {
this.sizeOfClique = sizeOfClique;
}
+ /**
+ * Set the clique vertexes
+ *
+ * @param cliques
+ * the list of vertexes -- can contain multiple cliques
+ */
public void setCliques(List<VLongWritable> cliques) {
this.cliques = cliques;
}
+ /**
+ * @return the size of the clique
+ */
public int getSizeOfClique() {
return sizeOfClique;
}
+ /**
+ * Get clique vertexes --- can contain multiple vertexes
+ *
+ * @return all the vertexes
+ */
+ public List<VLongWritable> getVertexes() {
+ return cliques;
+ }
+
+ /**
+ * rese the clique
+ */
public void reset() {
this.cliques.clear();
this.sizeOfClique = 0;
diff --git a/pregelix/pregelix-example/src/main/java/edu/uci/ics/pregelix/example/maximalclique/MaximalCliqueAggregator.java b/pregelix/pregelix-example/src/main/java/edu/uci/ics/pregelix/example/maximalclique/MaximalCliqueAggregator.java
new file mode 100644
index 0000000..d75de33
--- /dev/null
+++ b/pregelix/pregelix-example/src/main/java/edu/uci/ics/pregelix/example/maximalclique/MaximalCliqueAggregator.java
@@ -0,0 +1,65 @@
+package edu.uci.ics.pregelix.example.maximalclique;
+
+import org.apache.hadoop.io.NullWritable;
+
+import edu.uci.ics.hyracks.api.exceptions.HyracksDataException;
+import edu.uci.ics.pregelix.api.graph.GlobalAggregator;
+import edu.uci.ics.pregelix.api.graph.Vertex;
+import edu.uci.ics.pregelix.example.io.VLongWritable;
+
+/**
+ * The global aggregator aggregates the count of triangles
+ */
+public class MaximalCliqueAggregator
+ extends
+ GlobalAggregator<VLongWritable, CliquesWritable, NullWritable, AdjacencyListWritable, CliquesWritable, CliquesWritable> {
+
+ private CliquesWritable state = new CliquesWritable();
+
+ @Override
+ public void init() {
+
+ }
+
+ @Override
+ public void step(Vertex<VLongWritable, CliquesWritable, NullWritable, AdjacencyListWritable> v)
+ throws HyracksDataException {
+ CliquesWritable cliques = v.getVertexValue();
+ updateAggregateState(cliques);
+ }
+
+ /**
+ * Update the current aggregate state
+ *
+ * @param cliques the incoming cliques
+ */
+ private void updateAggregateState(CliquesWritable cliques) {
+ if (cliques.getSizeOfClique() > state.getSizeOfClique()) {
+ //reset the vertex state
+ state.reset();
+ state.setCliqueSize(cliques.getSizeOfClique());
+ state.setCliques(cliques.getVertexes());
+ } else if (cliques.getSizeOfClique() == state.getSizeOfClique()) {
+ //add the new cliques
+ state.getVertexes().addAll(cliques.getVertexes());
+ } else {
+ return;
+ }
+ }
+
+ @Override
+ public void step(CliquesWritable partialResult) {
+ updateAggregateState(partialResult);
+ }
+
+ @Override
+ public CliquesWritable finishPartial() {
+ return state;
+ }
+
+ @Override
+ public CliquesWritable finishFinal() {
+ return state;
+ }
+
+}
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 f7dcb47..2c0de14 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
@@ -24,6 +24,7 @@
import java.util.Map;
import java.util.TreeMap;
+import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.io.NullWritable;
import org.apache.hadoop.io.Text;
import org.apache.hadoop.mapreduce.RecordWriter;
@@ -34,8 +35,18 @@
import edu.uci.ics.pregelix.api.io.VertexWriter;
import edu.uci.ics.pregelix.api.io.text.TextVertexOutputFormat;
import edu.uci.ics.pregelix.api.io.text.TextVertexOutputFormat.TextVertexWriter;
+import edu.uci.ics.pregelix.api.job.PregelixJob;
+import edu.uci.ics.pregelix.api.util.BspUtils;
+import edu.uci.ics.pregelix.dataflow.util.IterationUtils;
+import edu.uci.ics.pregelix.example.client.Client;
import edu.uci.ics.pregelix.example.io.VLongWritable;
+import edu.uci.ics.pregelix.example.trianglecounting.TriangleCountingVertex;
+/**
+ * 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.
+ */
public class MaximalCliqueVertex extends Vertex<VLongWritable, CliquesWritable, NullWritable, AdjacencyListWritable> {
private Map<VLongWritable, AdjacencyListWritable> map = new TreeMap<VLongWritable, AdjacencyListWritable>();
@@ -46,6 +57,12 @@
private CliquesWritable tmpValue = new CliquesWritable();
private List<VLongWritable> cliques = new ArrayList<VLongWritable>();
+ /**
+ * Update the current maximal cliques
+ *
+ * @param values
+ * the received adjcency lists
+ */
private void updateCurrentMaximalCliques(Iterator<AdjacencyListWritable> values) {
map.clear();
vertexList.clear();
@@ -105,14 +122,19 @@
clique.set(keyIndex);
generateClique(clique);
tmpValue.setCliques(cliques);
- tmpValue.setClusterSize(clique.cardinality());
+ tmpValue.setCliqueSize(clique.cardinality());
}
-
+
//update the vertex state
setVertexValue(tmpValue);
}
- // output a clique with the bitmap representation
+ /**
+ * Output a clique with vertex ids.
+ *
+ * @param clique
+ * the bitmap representation of a clique
+ */
private void generateClique(BitSet clique) {
for (int j = 0; j < clique.length();) {
j = clique.nextSetBit(j);
@@ -122,6 +144,18 @@
}
}
+ /**
+ * find cliques using the depth-first search
+ *
+ * @param maxDepth
+ * the maximum search depth
+ * @param cliqueSoFar
+ * the the cliques found so far
+ * @param depthSoFar
+ * the current search depth
+ * @param currentSource
+ * the vertex to be added into the clique
+ */
private void searchClique(int maxDepth, BitSet cliqueSoFar, int depthSoFar, VLongWritable currentSource) {
if (depthSoFar > maxDepth) {
// update maximal clique info
@@ -149,7 +183,12 @@
updateMaximalClique(cliqueSoFar);
}
- // update the maximal clique to a larger one if it exists
+ /**
+ * Update the maximal clique to a larger one if it exists
+ *
+ * @param cliqueSoFar
+ * the clique so far, in the bitmap representation
+ */
private void updateMaximalClique(BitSet cliqueSoFar) {
int cliqueSize = cliqueSoFar.cardinality();
if (cliqueSize > largestCliqueSizeSoFar) {
@@ -163,7 +202,15 @@
}
}
- //should we test the vertex newVertex?
+ /**
+ * Should we test the vertex newVertex?
+ *
+ * @param newVertex
+ * the vertex to be tested
+ * @param cliqueSoFar
+ * the current clique, in the bitmap representation
+ * @return true if new vertex has been tested
+ */
private boolean isTested(VLongWritable newVertex, BitSet cliqueSoFar) {
int index = invertedMap.get(newVertex);
int largestSetIndex = cliqueSoFar.length() - 1;
@@ -177,7 +224,15 @@
}
}
- // will adding the newVertex yield a bigger clique?
+ /**
+ * Will adding the newVertex yield a bigger clique?
+ *
+ * @param newVertex
+ * the new vertex id
+ * @param cliqueSoFar
+ * the bitmap representation of the clique
+ * @return true if adding the new vertex yelds a bigger clique
+ */
private boolean isClique(VLongWritable newVertex, BitSet cliqueSoFar) {
AdjacencyListWritable adj = map.get(newVertex);
// check whether each existing vertex is in the neighbor set of newVertex
@@ -192,27 +247,59 @@
return true;
}
+ /**
+ * For superstep 1, send outgoing mesages.
+ * For superstep 2, calculate maximal cliques.
+ * otherwise, vote to halt.
+ */
@Override
public void compute(Iterator<AdjacencyListWritable> msgIterator) {
if (getSuperstep() == 1) {
sortEdges();
- sendMsg(getEdges());
+ sendOutgoingMsgs(getEdges());
} else if (getSuperstep() == 2) {
updateCurrentMaximalCliques(msgIterator);
} else {
voteToHalt();
}
}
-
+
@Override
public String toString() {
return getVertexId() + " " + getVertexValue();
}
- private void sendMsg(List<Edge<VLongWritable, NullWritable>> edges) {
+ private static CliquesWritable readMaximalCliqueResult(Configuration conf) {
+ try {
+ CliquesWritable result = (CliquesWritable) IterationUtils.readGlobalAggregateValue(conf,
+ BspUtils.getJobId(conf));
+ return result;
+ } catch (IOException e) {
+ throw new IllegalStateException(e);
+ }
+ }
+
+ public static void main(String[] args) throws Exception {
+ PregelixJob job = new PregelixJob(TriangleCountingVertex.class.getSimpleName());
+ job.setVertexClass(MaximalCliqueVertex.class);
+ job.setGlobalAggregatorClass(MaximalCliqueAggregator.class);
+ job.setDynamicVertexValueSize(true);
+ job.setVertexInputFormatClass(TextMaximalCliqueInputFormat.class);
+ job.setVertexOutputFormatClass(MaximalCliqueVertexOutputFormat.class);
+ Client.run(args, job);
+ System.out.println("maximal cliques: \n" + readMaximalCliqueResult(job.getConfiguration()));
+ }
+
+ /**
+ * Send the adjacency lists
+ *
+ * @param edges
+ * the outgoing edges
+ */
+ 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 adjListLong[0]
+ // 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.
getVertexValue().reset();
diff --git a/pregelix/pregelix-example/src/main/java/edu/uci/ics/pregelix/example/trianglecounting/TextTriangleCountingInputFormat.java b/pregelix/pregelix-example/src/main/java/edu/uci/ics/pregelix/example/trianglecounting/TextTriangleCountingInputFormat.java
new file mode 100644
index 0000000..bb399ff
--- /dev/null
+++ b/pregelix/pregelix-example/src/main/java/edu/uci/ics/pregelix/example/trianglecounting/TextTriangleCountingInputFormat.java
@@ -0,0 +1,111 @@
+/*
+ * Copyright 2009-2010 by The Regents of the University of California
+ * Licensed 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 from
+ *
+ * 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 edu.uci.ics.pregelix.example.trianglecounting;
+
+import java.io.IOException;
+import java.util.ArrayList;
+import java.util.List;
+
+import org.apache.hadoop.io.LongWritable;
+import org.apache.hadoop.io.Text;
+import org.apache.hadoop.mapreduce.InputSplit;
+import org.apache.hadoop.mapreduce.RecordReader;
+import org.apache.hadoop.mapreduce.TaskAttemptContext;
+
+import edu.uci.ics.pregelix.api.graph.Vertex;
+import edu.uci.ics.pregelix.api.io.VertexReader;
+import edu.uci.ics.pregelix.api.io.text.TextVertexInputFormat;
+import edu.uci.ics.pregelix.api.io.text.TextVertexInputFormat.TextVertexReader;
+import edu.uci.ics.pregelix.api.util.BspUtils;
+import edu.uci.ics.pregelix.example.io.VLongWritable;
+
+public class TextTriangleCountingInputFormat extends
+ TextVertexInputFormat<VLongWritable, VLongWritable, VLongWritable, VLongWritable> {
+
+ @Override
+ public VertexReader<VLongWritable, VLongWritable, VLongWritable, VLongWritable> createVertexReader(
+ InputSplit split, TaskAttemptContext context) throws IOException {
+ return new TextPageRankGraphReader(textInputFormat.createRecordReader(split, context));
+ }
+}
+
+@SuppressWarnings("rawtypes")
+class TextPageRankGraphReader extends TextVertexReader<VLongWritable, VLongWritable, VLongWritable, VLongWritable> {
+
+ private final static String separator = " ";
+ private Vertex vertex;
+ private VLongWritable vertexId = new VLongWritable();
+ private List<VLongWritable> pool = new ArrayList<VLongWritable>();
+ private int used = 0;
+
+ public TextPageRankGraphReader(RecordReader<LongWritable, Text> lineRecordReader) {
+ super(lineRecordReader);
+ }
+
+ @Override
+ public boolean nextVertex() throws IOException, InterruptedException {
+ return getRecordReader().nextKeyValue();
+ }
+
+ @SuppressWarnings("unchecked")
+ @Override
+ public Vertex<VLongWritable, VLongWritable, VLongWritable, VLongWritable> getCurrentVertex() throws IOException,
+ InterruptedException {
+ used = 0;
+ if (vertex == null)
+ vertex = (Vertex) BspUtils.createVertex(getContext().getConfiguration());
+ vertex.getMsgList().clear();
+ vertex.getEdges().clear();
+
+ vertex.reset();
+ Text line = getRecordReader().getCurrentValue();
+ String[] fields = line.toString().split(separator);
+
+ if (fields.length > 0) {
+ /**
+ * set the src vertex id
+ */
+ long src = Long.parseLong(fields[0]);
+ vertexId.set(src);
+ vertex.setVertexId(vertexId);
+ long dest = -1L;
+
+ /**
+ * set up edges
+ */
+ for (int i = 1; i < fields.length; i++) {
+ dest = Long.parseLong(fields[i]);
+ VLongWritable destId = allocate();
+ destId.set(dest);
+ vertex.addEdge(destId, null);
+ }
+ }
+ // vertex.sortEdges();
+ return vertex;
+ }
+
+ private VLongWritable allocate() {
+ if (used >= pool.size()) {
+ VLongWritable value = new VLongWritable();
+ pool.add(value);
+ used++;
+ return value;
+ } else {
+ VLongWritable value = pool.get(used);
+ used++;
+ return value;
+ }
+ }
+}
diff --git a/pregelix/pregelix-example/src/main/java/edu/uci/ics/pregelix/example/trianglecounting/TriangleCountingVertex.java b/pregelix/pregelix-example/src/main/java/edu/uci/ics/pregelix/example/trianglecounting/TriangleCountingVertex.java
index 2ee9d17..ed51782 100644
--- a/pregelix/pregelix-example/src/main/java/edu/uci/ics/pregelix/example/trianglecounting/TriangleCountingVertex.java
+++ b/pregelix/pregelix-example/src/main/java/edu/uci/ics/pregelix/example/trianglecounting/TriangleCountingVertex.java
@@ -23,6 +23,9 @@
import edu.uci.ics.pregelix.example.inputformat.TextPageRankInputFormat;
import edu.uci.ics.pregelix.example.io.VLongWritable;
+/**
+ * The triangle counting example -- counting the triangles in an undirected graph.
+ */
public class TriangleCountingVertex extends Vertex<VLongWritable, VLongWritable, VLongWritable, VLongWritable> {
private VLongWritable tmpValue = new VLongWritable(0);
@@ -95,7 +98,7 @@
new Text(vertex.getVertexValue().toString()));
}
}
-
+
@Override
public String toString() {
return getVertexId() + " " + getVertexValue();
@@ -137,6 +140,9 @@
}
}
+/**
+ * The comparator for Edge<VLongWritable, VLongWritable>.
+ */
class EdgeComparator implements Comparator<Edge<VLongWritable, VLongWritable>> {
@Override
diff --git a/pregelix/pregelix-example/src/test/java/edu/uci/ics/pregelix/example/dataload/DataLoadTest.java b/pregelix/pregelix-example/src/test/java/edu/uci/ics/pregelix/example/dataload/DataLoadTest.java
index 7787347..37f03a5 100644
--- a/pregelix/pregelix-example/src/test/java/edu/uci/ics/pregelix/example/dataload/DataLoadTest.java
+++ b/pregelix/pregelix-example/src/test/java/edu/uci/ics/pregelix/example/dataload/DataLoadTest.java
@@ -37,7 +37,7 @@
import edu.uci.ics.pregelix.core.jobgen.clusterconfig.ClusterConfig;
import edu.uci.ics.pregelix.core.util.PregelixHyracksIntegrationUtil;
import edu.uci.ics.pregelix.example.PageRankVertex;
-import edu.uci.ics.pregelix.example.PageRankVertex.SimplePageRankVertexInputFormat;
+import edu.uci.ics.pregelix.example.PageRankVertex.SimulatedPageRankVertexInputFormat;
import edu.uci.ics.pregelix.example.util.TestUtils;
@SuppressWarnings("deprecation")
@@ -65,7 +65,7 @@
public DataLoadTest() throws Exception {
job = new PregelixJob(GIRAPH_JOB_NAME);
job.setVertexClass(PageRankVertex.class);
- job.setVertexInputFormatClass(SimplePageRankVertexInputFormat.class);
+ job.setVertexInputFormatClass(SimulatedPageRankVertexInputFormat.class);
job.getConfiguration().setClass(PregelixJob.VERTEX_INDEX_CLASS, LongWritable.class, WritableComparable.class);
job.getConfiguration().setClass(PregelixJob.VERTEX_VALUE_CLASS, DoubleWritable.class, Writable.class);
job.getConfiguration().setClass(PregelixJob.EDGE_VALUE_CLASS, FloatWritable.class, Writable.class);
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 09684d2..b131edc 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
@@ -26,8 +26,8 @@
import edu.uci.ics.pregelix.example.ConnectedComponentsVertex;
import edu.uci.ics.pregelix.example.ConnectedComponentsVertex.SimpleConnectedComponentsVertexOutputFormat;
import edu.uci.ics.pregelix.example.PageRankVertex;
-import edu.uci.ics.pregelix.example.PageRankVertex.SimplePageRankVertexInputFormat;
import edu.uci.ics.pregelix.example.PageRankVertex.SimplePageRankVertexOutputFormat;
+import edu.uci.ics.pregelix.example.PageRankVertex.SimulatedPageRankVertexInputFormat;
import edu.uci.ics.pregelix.example.ReachabilityVertex;
import edu.uci.ics.pregelix.example.ReachabilityVertex.SimpleReachibilityVertexOutputFormat;
import edu.uci.ics.pregelix.example.ShortestPathsVertex;
@@ -35,6 +35,7 @@
import edu.uci.ics.pregelix.example.inputformat.TextPageRankInputFormat;
import edu.uci.ics.pregelix.example.inputformat.TextReachibilityVertexInputFormat;
import edu.uci.ics.pregelix.example.inputformat.TextShortestPathsInputFormat;
+import edu.uci.ics.pregelix.example.maximalclique.MaximalCliqueAggregator;
import edu.uci.ics.pregelix.example.maximalclique.MaximalCliqueVertex;
import edu.uci.ics.pregelix.example.maximalclique.MaximalCliqueVertex.MaximalCliqueVertexOutputFormat;
import edu.uci.ics.pregelix.example.maximalclique.TextMaximalCliqueInputFormat;
@@ -157,7 +158,7 @@
private static void generatePageRankJob(String jobName, String outputPath) throws IOException {
PregelixJob job = new PregelixJob(jobName);
job.setVertexClass(PageRankVertex.class);
- job.setVertexInputFormatClass(SimplePageRankVertexInputFormat.class);
+ job.setVertexInputFormatClass(SimulatedPageRankVertexInputFormat.class);
job.setMessageCombinerClass(PageRankVertex.SimpleSumCombiner.class);
job.setVertexOutputFormatClass(SimplePageRankVertexOutputFormat.class);
FileInputFormat.setInputPaths(job, HDFS_INPUTPATH);
@@ -169,7 +170,7 @@
private static void generateShortestPathJob(String jobName, String outputPath) throws IOException {
PregelixJob job = new PregelixJob(jobName);
job.setVertexClass(ShortestPathsVertex.class);
- job.setVertexInputFormatClass(SimplePageRankVertexInputFormat.class);
+ job.setVertexInputFormatClass(SimulatedPageRankVertexInputFormat.class);
job.setMessageCombinerClass(ShortestPathsVertex.SimpleMinCombiner.class);
job.setVertexOutputFormatClass(SimplePageRankVertexOutputFormat.class);
FileInputFormat.setInputPaths(job, HDFS_INPUTPATH);
@@ -206,11 +207,12 @@
private static void generateMaximalCliqueJob(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_INPUTPATH3);
FileOutputFormat.setOutputPath(job, new Path(HDFS_OUTPUTPAH3));
- job.setDynamicVertexValueSize(true);
job.getConfiguration().writeXml(new FileOutputStream(new File(outputPath)));
}
diff --git a/pregelix/pregelix-example/src/test/resources/jobs/MaximalClique.xml b/pregelix/pregelix-example/src/test/resources/jobs/MaximalClique.xml
index ab4b630..616c647 100644
--- a/pregelix/pregelix-example/src/test/resources/jobs/MaximalClique.xml
+++ b/pregelix/pregelix-example/src/test/resources/jobs/MaximalClique.xml
@@ -122,6 +122,7 @@
<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>
diff --git a/pregelix/pregelix-example/src/test/resources/jobs/PageRank.xml b/pregelix/pregelix-example/src/test/resources/jobs/PageRank.xml
index e425b38..744e5b0 100644
--- a/pregelix/pregelix-example/src/test/resources/jobs/PageRank.xml
+++ b/pregelix/pregelix-example/src/test/resources/jobs/PageRank.xml
@@ -123,7 +123,7 @@
<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.PageRankVertex$SimplePageRankVertexInputFormat</value></property>
+<property><name>pregelix.vertexInputFormatClass</name><value>edu.uci.ics.pregelix.example.PageRankVertex$SimulatedPageRankVertexInputFormat</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>mapred.reduce.slowstart.completed.maps</name><value>0.05</value></property>
diff --git a/pregelix/pregelix-example/src/test/resources/jobs/ShortestPaths.xml b/pregelix/pregelix-example/src/test/resources/jobs/ShortestPaths.xml
index 3719247..9e791e2 100644
--- a/pregelix/pregelix-example/src/test/resources/jobs/ShortestPaths.xml
+++ b/pregelix/pregelix-example/src/test/resources/jobs/ShortestPaths.xml
@@ -124,7 +124,7 @@
<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.PageRankVertex$SimplePageRankVertexInputFormat</value></property>
+<property><name>pregelix.vertexInputFormatClass</name><value>edu.uci.ics.pregelix.example.PageRankVertex$SimulatedPageRankVertexInputFormat</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>mapred.reduce.slowstart.completed.maps</name><value>0.05</value></property>