提交 e2f28d47 编写于 作者: V vasia

[FLINK-1523] [gelly] added getIn/Out degrees methods in update and messaging functions;

    deleted VertexWithValue type;
    deleted InaccessibleMethodException; if the options are not set, -1 is returned;
    added missing javadocs;
    added tests;
    renamed type parameters VertexKey -> K, VertexValue -> VV, EdgeValue -> EV.
上级 e1720673
/*
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.flink.graph;
/**
* The exception that gets thrown when the degree option or the number of vertices
* option in {@link org.apache.flink.graph.spargel.IterationConfiguration} was not set.
*/
public class InaccessibleMethodException extends Exception {
public InaccessibleMethodException() {}
public InaccessibleMethodException(String text) {
super(text);
}
}
/*
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.flink.graph;
import java.io.Serializable;
/**
* Represents the graph's nodes. It carries an ID and a value as well as the vertex inDegree and outDegree.
* For vertices with no value, use {@link org.apache.flink.types.NullValue} as the value type.
*
* @param <K>
* @param <V>
*/
public class VertexWithDegrees<K extends Comparable<K> & Serializable, V extends Serializable>
extends Vertex<K, V> {
private long inDegree;
private long outDegree;
public VertexWithDegrees() {
super();
inDegree = -1l;
outDegree = -1l;
}
public VertexWithDegrees(K k, V v) {
super(k,v);
inDegree = 0l;
outDegree = 0l;
}
public Long getInDegree() throws Exception{
if(inDegree == -1) {
throw new InaccessibleMethodException("The degree option was not set. To access the degrees, " +
"call iterationConfiguration.setOptDegrees(true).");
}
return inDegree;
}
public void setInDegree(Long inDegree) {
this.inDegree = inDegree;
}
public Long getOutDegree() throws Exception{
if(outDegree == -1) {
throw new InaccessibleMethodException("The degree option was not set. To access the degrees, " +
"call iterationConfiguration.setOptDegrees(true).");
}
return outDegree;
}
public void setOutDegree(Long outDegree) {
this.outDegree = outDegree;
}
}
......@@ -26,11 +26,10 @@ import org.apache.flink.graph.Edge;
import org.apache.flink.graph.EdgeDirection;
import org.apache.flink.graph.Graph;
import org.apache.flink.graph.Vertex;
import org.apache.flink.graph.VertexWithDegrees;
import org.apache.flink.graph.example.utils.IncrementalSSSPData;
import org.apache.flink.graph.spargel.IterationConfiguration;
import org.apache.flink.graph.spargel.MessageIterator;
import org.apache.flink.graph.spargel.MessagingFunction;
import org.apache.flink.graph.spargel.VertexCentricConfiguration;
import org.apache.flink.graph.spargel.VertexUpdateFunction;
import org.apache.flink.graph.utils.Tuple2ToVertexMap;
import org.apache.flink.graph.utils.Tuple3ToEdgeMap;
......@@ -95,7 +94,7 @@ public class IncrementalSSSPExample implements ProgramDescription {
graph.removeEdge(edgeToBeRemoved);
// configure the iteration
IterationConfiguration parameters = new IterationConfiguration();
VertexCentricConfiguration parameters = new VertexCentricConfiguration();
if(isInSSSP(edgeToBeRemoved, edgesInSSSP)) {
......@@ -160,7 +159,7 @@ public class IncrementalSSSPExample implements ProgramDescription {
@Override
public void updateVertex(Vertex<Long, Double> vertex, MessageIterator<Double> inMessages) throws Exception {
if (inMessages.hasNext()) {
Long outDegree = ((VertexWithDegrees)vertex).getOutDegree() - 1;
Long outDegree = getOutDegree() - 1;
// check if the vertex has another SP-Edge
if (outDegree > 0) {
// there is another shortest path from the source to this vertex
......
......@@ -20,6 +20,7 @@ package org.apache.flink.graph.library;
import org.apache.flink.graph.Graph;
import org.apache.flink.graph.GraphAlgorithm;
import org.apache.flink.graph.Vertex;
import org.apache.flink.graph.spargel.MessageIterator;
import org.apache.flink.graph.spargel.MessagingFunction;
import org.apache.flink.graph.spargel.VertexUpdateFunction;
......@@ -60,7 +61,7 @@ public class ConnectedComponentsAlgorithm implements GraphAlgorithm<Long, Long,
public static final class CCUpdater extends VertexUpdateFunction<Long, Long, Long> {
@Override
public void updateVertex(Long id, Long currentMin, MessageIterator<Long> messages) throws Exception {
public void updateVertex(Vertex<Long, Long> vertex, MessageIterator<Long> messages) throws Exception {
long min = Long.MAX_VALUE;
for (long msg : messages) {
......@@ -68,7 +69,7 @@ public class ConnectedComponentsAlgorithm implements GraphAlgorithm<Long, Long,
}
// update vertex value, if new minimum
if (min < currentMin) {
if (min < vertex.getValue()) {
setNewVertexValue(min);
}
}
......@@ -80,9 +81,9 @@ public class ConnectedComponentsAlgorithm implements GraphAlgorithm<Long, Long,
public static final class CCMessenger extends MessagingFunction<Long, Long, Long, NullValue> {
@Override
public void sendMessages(Long id, Long currentMin) throws Exception {
public void sendMessages(Vertex<Long, Long> vertex) throws Exception {
// send current minimum to neighbors
sendMessageToAllNeighbors(currentMin);
sendMessageToAllNeighbors(vertex.getValue());
}
}
}
......@@ -26,24 +26,21 @@ import org.apache.flink.api.common.aggregators.Aggregator;
import org.apache.flink.api.common.functions.IterationRuntimeContext;
import org.apache.flink.api.java.tuple.Tuple;
import org.apache.flink.api.java.tuple.Tuple2;
import org.apache.flink.api.java.tuple.Tuple3;
import org.apache.flink.graph.Edge;
import org.apache.flink.graph.EdgeDirection;
import org.apache.flink.graph.InaccessibleMethodException;
import org.apache.flink.graph.Vertex;
import org.apache.flink.graph.VertexWithDegrees;
import org.apache.flink.types.Value;
import org.apache.flink.util.Collector;
/**
* The base class for functions that produce messages between vertices as a part of a {@link VertexCentricIteration}.
*
* @param <VertexKey> The type of the vertex key (the vertex identifier).
* @param <VertexValue> The type of the vertex value (the state of the vertex).
* @param <K> The type of the vertex key (the vertex identifier).
* @param <VV> The type of the vertex value (the state of the vertex).
* @param <Message> The type of the message sent between vertices along the edges.
* @param <EdgeValue> The type of the values that are associated with the edges.
* @param <EV> The type of the values that are associated with the edges.
*/
public abstract class MessagingFunction<VertexKey, VertexValue, Message, EdgeValue> implements Serializable {
public abstract class MessagingFunction<K, VV, Message, EV> implements Serializable {
private static final long serialVersionUID = 1L;
......@@ -54,11 +51,12 @@ public abstract class MessagingFunction<VertexKey, VertexValue, Message, EdgeVal
private long numberOfVertices = -1L;
public long getNumberOfVertices() throws Exception{
if (numberOfVertices == -1) {
throw new InaccessibleMethodException("The number of vertices option is not set. " +
"To access the number of vertices, call iterationConfiguration.setOptNumVertices(true).");
}
/**
* Retrieves the number of vertices in the graph.
* @return the number of vertices if the {@link IterationConfiguration#setOptNumVertices(boolean)}
* option has been set; -1 otherwise.
*/
public long getNumberOfVertices() {
return numberOfVertices;
}
......@@ -73,11 +71,15 @@ public abstract class MessagingFunction<VertexKey, VertexValue, Message, EdgeVal
private EdgeDirection direction;
/**
* Retrieves the edge direction in which messages are propagated in the vertex-centric iteration.
* @return the messaging {@link EdgeDirection}
*/
public EdgeDirection getDirection() {
return direction;
}
public void setDirection(EdgeDirection direction) {
void setDirection(EdgeDirection direction) {
this.direction = direction;
}
......@@ -93,7 +95,7 @@ public abstract class MessagingFunction<VertexKey, VertexValue, Message, EdgeVal
*
* @throws Exception The computation may throw exceptions, which causes the superstep to fail.
*/
public abstract void sendMessages(Vertex<VertexKey, VertexValue> vertex) throws Exception;
public abstract void sendMessages(Vertex<K, VV> vertex) throws Exception;
/**
* This method is executed one per superstep before the vertex update function is invoked for each vertex.
......@@ -117,12 +119,12 @@ public abstract class MessagingFunction<VertexKey, VertexValue, Message, EdgeVal
* @return An iterator with all outgoing edges.
*/
@SuppressWarnings("unchecked")
public Iterable<Edge<VertexKey, EdgeValue>> getEdges() {
public Iterable<Edge<K, EV>> getEdges() {
if (edgesUsed) {
throw new IllegalStateException("Can use either 'getEdges()' or 'sendMessageToAllTargets()' exactly once.");
}
edgesUsed = true;
this.edgeIterator.set((Iterator<Edge<VertexKey, EdgeValue>>) edges);
this.edgeIterator.set((Iterator<Edge<K, EV>>) edges);
return this.edgeIterator;
}
......@@ -143,7 +145,7 @@ public abstract class MessagingFunction<VertexKey, VertexValue, Message, EdgeVal
while (edges.hasNext()) {
Tuple next = (Tuple) edges.next();
VertexKey k = next.getField(1);
K k = next.getField(1);
outValue.f0 = k;
out.collect(outValue);
}
......@@ -156,7 +158,7 @@ public abstract class MessagingFunction<VertexKey, VertexValue, Message, EdgeVal
* @param target The key (id) of the target vertex to message.
* @param m The message.
*/
public void sendMessageTo(VertexKey target, Message m) {
public void sendMessageTo(K target, Message m) {
outValue.f0 = target;
outValue.f1 = m;
out.collect(outValue);
......@@ -210,39 +212,42 @@ public abstract class MessagingFunction<VertexKey, VertexValue, Message, EdgeVal
// internal methods and state
// --------------------------------------------------------------------------------------------
private Tuple2<VertexKey, Message> outValue;
private Tuple2<K, Message> outValue;
private IterationRuntimeContext runtimeContext;
private Iterator<?> edges;
private Collector<Tuple2<VertexKey, Message>> out;
private Collector<Tuple2<K, Message>> out;
private EdgesIterator<VertexKey, EdgeValue> edgeIterator;
private EdgesIterator<K, EV> edgeIterator;
private boolean edgesUsed;
private long inDegree = -1;
private long outDegree = -1;
void init(IterationRuntimeContext context) {
this.runtimeContext = context;
this.outValue = new Tuple2<VertexKey, Message>();
this.edgeIterator = new EdgesIterator<VertexKey, EdgeValue>();
this.outValue = new Tuple2<K, Message>();
this.edgeIterator = new EdgesIterator<K, EV>();
}
void set(Iterator<?> edges, Collector<Tuple2<VertexKey, Message>> out) {
void set(Iterator<?> edges, Collector<Tuple2<K, Message>> out) {
this.edges = edges;
this.out = out;
this.edgesUsed = false;
}
private static final class EdgesIterator<VertexKey, EdgeValue>
implements Iterator<Edge<VertexKey, EdgeValue>>, Iterable<Edge<VertexKey, EdgeValue>>
private static final class EdgesIterator<K, EV>
implements Iterator<Edge<K, EV>>, Iterable<Edge<K, EV>>
{
private Iterator<Edge<VertexKey, EdgeValue>> input;
private Iterator<Edge<K, EV>> input;
private Edge<VertexKey, EdgeValue> edge = new Edge<VertexKey, EdgeValue>();
private Edge<K, EV> edge = new Edge<K, EV>();
void set(Iterator<Edge<VertexKey, EdgeValue>> input) {
void set(Iterator<Edge<K, EV>> input) {
this.input = input;
}
......@@ -252,8 +257,8 @@ public abstract class MessagingFunction<VertexKey, VertexValue, Message, EdgeVal
}
@Override
public Edge<VertexKey, EdgeValue> next() {
Edge<VertexKey, EdgeValue> next = input.next();
public Edge<K, EV> next() {
Edge<K, EV> next = input.next();
edge.setSource(next.f0);
edge.setTarget(next.f1);
edge.setValue(next.f2);
......@@ -265,28 +270,34 @@ public abstract class MessagingFunction<VertexKey, VertexValue, Message, EdgeVal
throw new UnsupportedOperationException();
}
@Override
public Iterator<Edge<VertexKey, EdgeValue>> iterator() {
public Iterator<Edge<K, EV>> iterator() {
return this;
}
}
/**
* In order to hide the Tuple3(actualValue, inDegree, outDegree) vertex value from the user,
* another function will be called from {@link org.apache.flink.graph.spargel.VertexCentricIteration}.
*
* This function will retrieve the vertex from the vertexState and will set its degrees, afterwards calling
* the regular sendMessages function.
*
* @param newVertexState
* @throws Exception
* Retrieves the vertex in-degree (number of in-coming edges).
* @return The in-degree of this vertex if the {@link IterationConfiguration#setOptDegrees(boolean)}
* option has been set; -1 otherwise.
*/
void sendMessagesFromVertexCentricIteration(Vertex<VertexKey, Tuple3<VertexValue, Long, Long>> newVertexState)
throws Exception {
VertexWithDegrees<VertexKey, VertexValue> vertex = new VertexWithDegrees<VertexKey, VertexValue>(newVertexState.getId(),
newVertexState.getValue().f0);
vertex.setInDegree(newVertexState.getValue().f1);
vertex.setOutDegree(newVertexState.getValue().f2);
sendMessages(vertex);
public long getInDegree() {
return inDegree;
}
void setInDegree(long inDegree) {
this.inDegree = inDegree;
}
/**
* Retrieve the vertex out-degree (number of out-going edges).
* @return The out-degree of this vertex if the {@link IterationConfiguration#setOptDegrees(boolean)}
* option has been set; -1 otherwise.
*/
public long getOutDegree() {
return outDegree;
}
void setOutDegree(long outDegree) {
this.outDegree = outDegree;
}
}
......@@ -24,9 +24,7 @@ import java.util.Collection;
import org.apache.flink.api.common.aggregators.Aggregator;
import org.apache.flink.api.common.functions.IterationRuntimeContext;
import org.apache.flink.api.java.tuple.Tuple3;
import org.apache.flink.graph.InaccessibleMethodException;
import org.apache.flink.graph.Vertex;
import org.apache.flink.graph.VertexWithDegrees;
import org.apache.flink.types.Value;
import org.apache.flink.util.Collector;
......@@ -35,11 +33,11 @@ import org.apache.flink.util.Collector;
* incoming messages. The central method is {@link #updateVertex(Comparable, Object, MessageIterator)}, which is
* invoked once per vertex per superstep.
*
* <VertexKey> The vertex key type.
* <VertexValue> The vertex value type.
* <K> The vertex key type.
* <VV> The vertex value type.
* <Message> The message type.
*/
public abstract class VertexUpdateFunction<VertexKey, VertexValue, Message> implements Serializable {
public abstract class VertexUpdateFunction<K, VV, Message> implements Serializable {
private static final long serialVersionUID = 1L;
......@@ -50,11 +48,12 @@ public abstract class VertexUpdateFunction<VertexKey, VertexValue, Message> impl
private long numberOfVertices = -1L;
public long getNumberOfVertices() throws Exception{
if (numberOfVertices == -1) {
throw new InaccessibleMethodException("The number of vertices option is not set. " +
"To access the number of vertices, call iterationConfiguration.setOptNumVertices(true).");
}
/**
* Retrieves the number of vertices in the graph.
* @return the number of vertices if the {@link IterationConfiguration#setOptNumVertices(boolean)}
* option has been set; -1 otherwise.
*/
public long getNumberOfVertices() {
return numberOfVertices;
}
......@@ -66,7 +65,7 @@ public abstract class VertexUpdateFunction<VertexKey, VertexValue, Message> impl
private boolean optDegrees;
public boolean isOptDegrees() {
boolean isOptDegrees() {
return optDegrees;
}
......@@ -80,7 +79,7 @@ public abstract class VertexUpdateFunction<VertexKey, VertexValue, Message> impl
/**
* This method is invoked once per vertex per superstep. It receives the current state of the vertex, as well as
* the incoming messages. It may set a new vertex state via {@link #setNewVertexValue(Object)}. If the vertex
* the incoming messages. It may set a new vertex state via {@link #setNewVV(Object)}. If the vertex
* state is changed, it will trigger the sending of messages via the {@link MessagingFunction}.
*
* @param vertex The vertex.
......@@ -88,7 +87,7 @@ public abstract class VertexUpdateFunction<VertexKey, VertexValue, Message> impl
*
* @throws Exception The computation may throw exceptions, which causes the superstep to fail.
*/
public abstract void updateVertex(Vertex<VertexKey, VertexValue> vertex, MessageIterator<Message> inMessages) throws Exception;
public abstract void updateVertex(Vertex<K, VV> vertex, MessageIterator<Message> inMessages) throws Exception;
/**
* This method is executed one per superstep before the vertex update function is invoked for each vertex.
......@@ -109,7 +108,7 @@ public abstract class VertexUpdateFunction<VertexKey, VertexValue, Message> impl
*
* @param newValue The new vertex value.
*/
public void setNewVertexValue(VertexValue newValue) {
public void setNewVertexValue(VV newValue) {
if(isOptDegrees()) {
outValWithDegrees.f1.f0 = newValue;
outWithDegrees.collect(outValWithDegrees);
......@@ -167,30 +166,58 @@ public abstract class VertexUpdateFunction<VertexKey, VertexValue, Message> impl
private IterationRuntimeContext runtimeContext;
private Collector<Vertex<VertexKey, VertexValue>> out;
private Collector<Vertex<K, VV>> out;
private Collector<Vertex<VertexKey, Tuple3<VertexValue, Long, Long>>> outWithDegrees;
private Collector<Vertex<K, Tuple3<VV, Long, Long>>> outWithDegrees;
private Vertex<VertexKey, VertexValue> outVal;
private Vertex<K, VV> outVal;
private Vertex<VertexKey, Tuple3<VertexValue, Long, Long>> outValWithDegrees;
private Vertex<K, Tuple3<VV, Long, Long>> outValWithDegrees;
private long inDegree = -1;
private long outDegree = -1;
void init(IterationRuntimeContext context) {
this.runtimeContext = context;
}
void setOutput(Vertex<K, VV> outVal, Collector<Vertex<K, VV>> out) {
this.outVal = outVal;
this.out = out;
}
@SuppressWarnings({ "unchecked", "rawtypes" })
<ValueWithDegree> void setOutputWithDegrees(Vertex<K, ValueWithDegree> outVal,
Collector out) {
this.outValWithDegrees = (Vertex<K, Tuple3<VV, Long, Long>>) outVal;
this.outWithDegrees = out;
}
void setOutputWithDegrees(Vertex<VertexKey, Tuple3<VertexValue, Long, Long>> outValWithDegrees,
Collector<Vertex<VertexKey, Tuple3<VertexValue, Long, Long>>> outWithDegrees) {
this.outValWithDegrees = outValWithDegrees;
this.outWithDegrees = outWithDegrees;
/**
* Retrieves the vertex in-degree (number of in-coming edges).
* @return The in-degree of this vertex if the {@link IterationConfiguration#setOptDegrees(boolean)}
* option has been set; -1 otherwise.
*/
public long getInDegree() {
return inDegree;
}
void setOutput(Vertex<VertexKey, VertexValue> outVal, Collector<Vertex<VertexKey, VertexValue>> out) {
this.outVal = outVal;
this.out = out;
void setInDegree(long inDegree) {
this.inDegree = inDegree;
}
/**
* Retrieve the vertex out-degree (number of out-going edges).
* @return The out-degree of this vertex if the {@link IterationConfiguration#setOptDegrees(boolean)}
* option has been set; -1 otherwise.
*/
public long getOutDegree() {
return outDegree;
}
void setOutDegree(long outDegree) {
this.outDegree = outDegree;
}
/**
......@@ -204,12 +231,12 @@ public abstract class VertexUpdateFunction<VertexKey, VertexValue, Message> impl
* @param inMessages
* @throws Exception
*/
void updateVertexFromVertexCentricIteration(Vertex<VertexKey, Tuple3<VertexValue, Long, Long>> vertexState,
@SuppressWarnings("unchecked")
<VertexWithDegree> void updateVertexFromVertexCentricIteration(Vertex<K, VertexWithDegree> vertexState,
MessageIterator<Message> inMessages) throws Exception {
VertexWithDegrees<VertexKey, VertexValue> vertex = new VertexWithDegrees<VertexKey, VertexValue>(vertexState.getId(),
vertexState.getValue().f0);
vertex.setInDegree(vertexState.getValue().f1);
vertex.setOutDegree(vertexState.getValue().f2);
Vertex<K, VV> vertex = new Vertex<K, VV>(vertexState.f0,
((Tuple3<VV, Long, Long>)vertexState.getValue()).f0);
updateVertex(vertex, inMessages);
}
......
......@@ -22,7 +22,6 @@ import org.apache.flink.api.common.functions.MapFunction;
import org.apache.flink.api.java.ExecutionEnvironment;
import org.apache.flink.api.java.io.DiscardingOutputFormat;
import org.apache.flink.api.java.tuple.Tuple2;
import org.apache.flink.api.java.tuple.Tuple3;
import org.apache.flink.graph.Graph;
import org.apache.flink.graph.Vertex;
import org.apache.flink.graph.spargel.MessageIterator;
......
......@@ -31,7 +31,6 @@ import org.apache.flink.graph.gsa.GatherFunction;
import org.apache.flink.graph.gsa.GatherSumApplyIteration;
import org.apache.flink.graph.gsa.Neighbor;
import org.apache.flink.graph.gsa.SumFunction;
import org.apache.flink.graph.IterationConfiguration;
import org.apache.flink.test.util.MultipleProgramsTestBase;
import org.apache.flink.types.LongValue;
import org.junit.After;
......
......@@ -26,13 +26,10 @@ import org.apache.flink.api.common.functions.MapFunction;
import org.apache.flink.api.java.DataSet;
import org.apache.flink.api.java.ExecutionEnvironment;
import org.apache.flink.api.java.tuple.Tuple1;
import org.apache.flink.api.java.tuple.Tuple2;
import org.apache.flink.api.java.tuple.Tuple3;
import org.apache.flink.graph.Edge;
import org.apache.flink.graph.EdgeDirection;
import org.apache.flink.graph.Graph;
import org.apache.flink.graph.Vertex;
import org.apache.flink.graph.IterationConfiguration;
import org.apache.flink.graph.spargel.MessageIterator;
import org.apache.flink.graph.spargel.MessagingFunction;
import org.apache.flink.graph.spargel.VertexCentricConfiguration;
......@@ -48,6 +45,7 @@ import org.junit.Test;
import org.junit.rules.TemporaryFolder;
import org.junit.runner.RunWith;
import org.junit.runners.Parameterized;
import org.apache.flink.graph.utils.VertexToTuple2Map;
@RunWith(Parameterized.class)
public class VertexCentricConfigurationITCase extends MultipleProgramsTestBase {
......@@ -88,6 +86,7 @@ public class VertexCentricConfigurationITCase extends MultipleProgramsTestBase {
parameters.addBroadcastSetForUpdateFunction("updateBcastSet", env.fromElements(1, 2, 3));
parameters.addBroadcastSetForMessagingFunction("messagingBcastSet", env.fromElements(4, 5, 6));
parameters.registerAggregator("superstepAggregator", new LongSumAggregator());
parameters.setOptNumVertices(true);
Graph<Long, Long, Long> result = graph.runVertexCentricIteration(
new UpdateFunction(), new MessageFunction(), 10, parameters);
......@@ -135,6 +134,29 @@ public class VertexCentricConfigurationITCase extends MultipleProgramsTestBase {
"5 15";
}
@Test
public void testDefaultConfiguration() throws Exception {
/*
* Test Graph's runVertexCentricIteration when configuration parameters are not provided
* i.e. degrees and numVertices will be -1, EdgeDirection will be OUT.
*/
final ExecutionEnvironment env = ExecutionEnvironment.getExecutionEnvironment();
Graph<Long, Long, Long> graph = Graph.fromCollection(TestGraphUtils.getLongLongVertices(),
TestGraphUtils.getLongLongEdges(), env).mapVertices(new AssignOneMapper());
Graph<Long, Long, Long> result = graph.runVertexCentricIteration(
new UpdateFunctionDefault(), new MessageFunctionDefault(), 5);
result.getVertices().map(new VertexToTuple2Map<Long, Long>()).writeAsCsv(resultPath, "\n", "\t");
env.execute();
expectedResult = "1 6\n" +
"2 6\n" +
"3 6\n" +
"4 6\n" +
"5 6";
}
@Test
public void testIterationDefaultDirection() throws Exception {
......@@ -176,7 +198,7 @@ public class VertexCentricConfigurationITCase extends MultipleProgramsTestBase {
.mapVertices(new InitialiseHashSetMapper());
// configure the iteration
IterationConfiguration parameters = new IterationConfiguration();
VertexCentricConfiguration parameters = new VertexCentricConfiguration();
parameters.setDirection(EdgeDirection.IN);
......@@ -208,7 +230,7 @@ public class VertexCentricConfigurationITCase extends MultipleProgramsTestBase {
.mapVertices(new InitialiseHashSetMapper());
// configure the iteration
IterationConfiguration parameters = new IterationConfiguration();
VertexCentricConfiguration parameters = new VertexCentricConfiguration();
parameters.setDirection(EdgeDirection.ALL);
......@@ -226,12 +248,37 @@ public class VertexCentricConfigurationITCase extends MultipleProgramsTestBase {
"5 [1, 3, 4]";
}
@Test
public void testNumVerticesNotSet() throws Exception {
/*
* Test that if the number of vertices option is not set, -1 is returned as value.
*/
final ExecutionEnvironment env = ExecutionEnvironment.getExecutionEnvironment();
Graph<Long, Long, Long> graph = Graph.fromCollection(TestGraphUtils.getLongLongVertices(),
TestGraphUtils.getLongLongEdges(), env);
DataSet<Vertex<Long, Long>> verticesWithNumVertices = graph.runVertexCentricIteration(new UpdateFunctionNumVertices(),
new DummyMessageFunction(), 2).getVertices();
verticesWithNumVertices.writeAsCsv(resultPath, "\n", "\t");
env.execute();
expectedResult = "1 -1\n" +
"2 -1\n" +
"3 -1\n" +
"4 -1\n" +
"5 -1";
}
@Test
public void testInDegreesSet() throws Exception {
/*
* Test that if the degrees are set, the in degrees can be accessed in every superstep and the value
* is correctly computed.
* Test that if the degrees are set, they can be accessed in every superstep
* inside the update function and the value
* is correctly computed for degrees in the messaging function.
*/
final ExecutionEnvironment env = ExecutionEnvironment.getExecutionEnvironment();
......@@ -239,14 +286,14 @@ public class VertexCentricConfigurationITCase extends MultipleProgramsTestBase {
TestGraphUtils.getLongLongEdges(), env);
// configure the iteration
IterationConfiguration parameters = new IterationConfiguration();
VertexCentricConfiguration parameters = new VertexCentricConfiguration();
parameters.setOptDegrees(true);
DataSet<Vertex<Long, Long>> verticesWithInDegree = graph.runVertexCentricIteration(new UpdateFunctionInDegree(),
new DummyMessageFunction(), 5, parameters).getVertices();
DataSet<Vertex<Long, Long>> verticesWithDegrees = graph.runVertexCentricIteration(
new UpdateFunctionInDegrees(), new DegreesMessageFunction(), 5, parameters).getVertices();
verticesWithInDegree.writeAsCsv(resultPath, "\n", "\t");
verticesWithDegrees.writeAsCsv(resultPath, "\n", "\t");
env.execute();
expectedResult = "1 1\n" +
......@@ -257,41 +304,36 @@ public class VertexCentricConfigurationITCase extends MultipleProgramsTestBase {
}
@Test
public void testOutDegreesSet() throws Exception {
public void testInDegreesNotSet() throws Exception {
/*
* Test that if the degrees are set, the out degrees can be accessed in every superstep and the value
* is correctly computed.
* Test that if the degrees option is not set, then -1 is returned as a value for in-degree.
*/
final ExecutionEnvironment env = ExecutionEnvironment.getExecutionEnvironment();
Graph<Long, Long, Long> graph = Graph.fromCollection(TestGraphUtils.getLongLongVertices(),
TestGraphUtils.getLongLongEdges(), env);
// configure the iteration
IterationConfiguration parameters = new IterationConfiguration();
parameters.setOptDegrees(true);
DataSet<Vertex<Long, Long>> verticesWithDegrees = graph.runVertexCentricIteration(
new UpdateFunctionInDegrees(), new DummyMessageFunction(), 2).getVertices();
DataSet<Vertex<Long, Long>> verticesWithOutDegree = graph.runVertexCentricIteration(new UpdateFunctionOutDegree(),
new DummyMessageFunction(), 5, parameters).getVertices();
verticesWithOutDegree.writeAsCsv(resultPath, "\n", "\t");
verticesWithDegrees.writeAsCsv(resultPath, "\n", "\t");
env.execute();
expectedResult = "1 2\n" +
"2 1\n" +
"3 2\n" +
"4 1\n" +
"5 1";
expectedResult = "1 -1\n" +
"2 -1\n" +
"3 -1\n" +
"4 -1\n" +
"5 -1";
}
@Test
public void testNumVerticesSet() throws Exception {
public void testOutDegreesSet() throws Exception {
/*
* Test that if the number of vertices option is set, it can be accessed in every superstep and the value
* is correctly computed.
* Test that if the degrees are set, they can be accessed in every superstep
* inside the update function and the value
* is correctly computed for degrees in the messaging function.
*/
final ExecutionEnvironment env = ExecutionEnvironment.getExecutionEnvironment();
......@@ -299,56 +341,45 @@ public class VertexCentricConfigurationITCase extends MultipleProgramsTestBase {
TestGraphUtils.getLongLongEdges(), env);
// configure the iteration
IterationConfiguration parameters = new IterationConfiguration();
VertexCentricConfiguration parameters = new VertexCentricConfiguration();
parameters.setOptNumVertices(true);
parameters.setOptDegrees(true);
DataSet<Vertex<Long, Long>> verticesWithNumVertices = graph.runVertexCentricIteration(new UpdateFunctionNumVertices(),
new DummyMessageFunction(), 5, parameters).getVertices();
DataSet<Vertex<Long, Long>> verticesWithDegrees = graph.runVertexCentricIteration(
new UpdateFunctionOutDegrees(), new DegreesMessageFunction(), 5, parameters).getVertices();
verticesWithNumVertices.writeAsCsv(resultPath, "\n", "\t");
verticesWithDegrees.writeAsCsv(resultPath, "\n", "\t");
env.execute();
expectedResult = "1 5\n" +
"2 5\n" +
"3 5\n" +
"4 5\n" +
"5 5";
expectedResult = "1 2\n" +
"2 1\n" +
"3 2\n" +
"4 1\n" +
"5 1";
}
@Test
public void testDegrees() throws Exception {
public void testOutDegreesNotSet() throws Exception {
/*
* Test that if the degrees are set, they can be accessed in every superstep and the value
* is correctly computed for both in and out degrees.
* Test that if the degrees option is not set, then -1 is returned as a value for out-degree.
*/
final ExecutionEnvironment env = ExecutionEnvironment.getExecutionEnvironment();
Graph<Long, Tuple3<Long, Long, Boolean>, Long> graph = Graph.fromCollection(TestGraphUtils.getLongVerticesWithDegrees(),
Graph<Long, Long, Long> graph = Graph.fromCollection(TestGraphUtils.getLongLongVertices(),
TestGraphUtils.getLongLongEdges(), env);
// configure the iteration
IterationConfiguration parameters = new IterationConfiguration();
parameters.setOptDegrees(true);
DataSet<Vertex<Long, Tuple3<Long, Long, Boolean>>> verticesWithDegrees = graph.runVertexCentricIteration(
new UpdateFunctionDegrees(), new DegreeMessageFunction(), 5, parameters).getVertices();
DataSet<Vertex<Long, Long>> verticesWithDegrees = graph.runVertexCentricIteration(
new UpdateFunctionInDegrees(), new DummyMessageFunction(), 2).getVertices();
verticesWithDegrees.map(new MapFunction<Vertex<Long,Tuple3<Long,Long,Boolean>>, Tuple2<Long, Boolean>>() {
@Override
public Tuple2<Long, Boolean> map(Vertex<Long, Tuple3<Long, Long, Boolean>> vertex) throws Exception {
return new Tuple2<Long, Boolean>(vertex.getId(), vertex.getValue().f2);
}
}).writeAsCsv(resultPath, "\n", "\t");
verticesWithDegrees.writeAsCsv(resultPath, "\n", "\t");
env.execute();
expectedResult = "1 true\n" +
"2 true\n" +
"3 true\n" +
"4 true\n" +
"5 true";
expectedResult = "1 -1\n" +
"2 -1\n" +
"3 -1\n" +
"4 -1\n" +
"5 -1";
}
@Test
......@@ -364,7 +395,7 @@ public class VertexCentricConfigurationITCase extends MultipleProgramsTestBase {
TestGraphUtils.getLongLongEdges(), env);
// configure the iteration
IterationConfiguration parameters = new IterationConfiguration();
VertexCentricConfiguration parameters = new VertexCentricConfiguration();
parameters.setOptDegrees(true);
parameters.setDirection(EdgeDirection.ALL);
......@@ -399,12 +430,36 @@ public class VertexCentricConfigurationITCase extends MultipleProgramsTestBase {
// test aggregator
aggregator = getIterationAggregator("superstepAggregator");
// test number of vertices
Assert.assertEquals(5, getNumberOfVertices());
}
@Override
public void updateVertex(Vertex<Long, Long> vertex, MessageIterator<Long> inMessages) {
long superstep = getSuperstepNumber();
aggregator.aggregate(superstep);
setNewVertexValue(vertex.getValue() + 1);
}
}
@SuppressWarnings("serial")
public static final class UpdateFunctionDefault extends VertexUpdateFunction<Long, Long, Long> {
LongSumAggregator aggregator = new LongSumAggregator();
@Override
public void updateVertex(Vertex<Long, Long> vertex, MessageIterator<Long> inMessages) {
// test number of vertices
Assert.assertEquals(-1, getNumberOfVertices());
// test degrees
Assert.assertEquals(-1, getInDegree());
Assert.assertEquals(-1, getOutDegree());
setNewVertexValue(vertex.getValue() + 1);
}
}
......@@ -422,6 +477,9 @@ public class VertexCentricConfigurationITCase extends MultipleProgramsTestBase {
Assert.assertEquals(5, bcastSet.get(1));
Assert.assertEquals(6, bcastSet.get(2));
// test number of vertices
Assert.assertEquals(5, getNumberOfVertices());
// test aggregator
if (getSuperstepNumber() == 2) {
long aggrValue = ((LongValue)getPreviousIterationAggregate("superstepAggregator")).getValue();
......@@ -437,28 +495,18 @@ public class VertexCentricConfigurationITCase extends MultipleProgramsTestBase {
}
@SuppressWarnings("serial")
public static final class UpdateFunctionInDegree extends VertexUpdateFunction<Long, Long, Long> {
public static final class MessageFunctionDefault extends MessagingFunction<Long, Long, Long, Long> {
@Override
public void updateVertex(Vertex<Long, Long> vertex, MessageIterator<Long> inMessages) {
try {
setNewVertexValue(((VertexWithDegrees) vertex).getInDegree());
} catch (Exception e) {
e.printStackTrace();
}
}
}
@SuppressWarnings("serial")
public static final class UpdateFunctionOutDegree extends VertexUpdateFunction<Long, Long, Long> {
public void sendMessages(Vertex<Long, Long> vertex) {
// test number of vertices
Assert.assertEquals(-1, getNumberOfVertices());
@Override
public void updateVertex(Vertex<Long, Long> vertex, MessageIterator<Long> inMessages) {
try {
setNewVertexValue(((VertexWithDegrees) vertex).getOutDegree());
} catch (Exception e) {
e.printStackTrace();
}
// test degrees
Assert.assertEquals(-1, getInDegree());
Assert.assertEquals(-1, getOutDegree());
//send message to keep vertices active
sendMessageToAllNeighbors(vertex.getValue());
}
}
......@@ -467,11 +515,7 @@ public class VertexCentricConfigurationITCase extends MultipleProgramsTestBase {
@Override
public void updateVertex(Vertex<Long, Long> vertex, MessageIterator<Long> inMessages) {
try {
setNewVertexValue(getNumberOfVertices());
} catch (Exception e) {
e.printStackTrace();
}
}
}
......@@ -495,8 +539,25 @@ public class VertexCentricConfigurationITCase extends MultipleProgramsTestBase {
}
@SuppressWarnings("serial")
public static final class VertexUpdateDirection extends VertexUpdateFunction<Long, HashSet<Long>,
Long> {
public static final class DegreesMessageFunction extends MessagingFunction<Long, Long, Long, Long> {
@Override
public void sendMessages(Vertex<Long, Long> vertex) {
if (vertex.getId().equals(1)) {
Assert.assertEquals(2, getOutDegree());
Assert.assertEquals(1, getInDegree());
}
else if(vertex.getId().equals(3)) {
Assert.assertEquals(2, getOutDegree());
Assert.assertEquals(2, getInDegree());
}
//send message to keep vertices active
sendMessageToAllNeighbors(vertex.getValue());
}
}
@SuppressWarnings("serial")
public static final class VertexUpdateDirection extends VertexUpdateFunction<Long, HashSet<Long>, Long> {
@Override
public void updateVertex(Vertex<Long, HashSet<Long>> vertex, MessageIterator<Long> messages) throws Exception {
......@@ -510,6 +571,26 @@ public class VertexCentricConfigurationITCase extends MultipleProgramsTestBase {
}
}
@SuppressWarnings("serial")
public static final class UpdateFunctionInDegrees extends VertexUpdateFunction<Long, Long, Long> {
@Override
public void updateVertex(Vertex<Long, Long> vertex, MessageIterator<Long> inMessages) {
long inDegree = getInDegree();
setNewVertexValue(inDegree);
}
}
@SuppressWarnings("serial")
public static final class UpdateFunctionOutDegrees extends VertexUpdateFunction<Long, Long, Long> {
@Override
public void updateVertex(Vertex<Long, Long> vertex, MessageIterator<Long> inMessages) {
long outDegree = getOutDegree();
setNewVertexValue(outDegree);
}
}
@SuppressWarnings("serial")
public static final class VertexUpdateNumNeighbors extends VertexUpdateFunction<Long, Boolean,
Long> {
......@@ -519,25 +600,21 @@ public class VertexCentricConfigurationITCase extends MultipleProgramsTestBase {
long count = 0;
for(long msg : messages) {
for(@SuppressWarnings("unused") long msg : messages) {
count++;
}
setNewVertexValue(count == (((VertexWithDegrees)vertex).getInDegree() + ((VertexWithDegrees)vertex).getOutDegree()));
setNewVertexValue(count == (getInDegree() + getOutDegree()));
}
}
@SuppressWarnings("serial")
public static final class UpdateFunctionDegrees extends VertexUpdateFunction<Long, Tuple3<Long, Long, Boolean>, Long> {
public static final class UpdateFunctionDegrees extends VertexUpdateFunction<Long, Long, Long> {
@Override
public void updateVertex(Vertex<Long, Tuple3<Long, Long, Boolean>> vertex, MessageIterator<Long> inMessages) {
try {
setNewVertexValue(new Tuple3(vertex.getValue().f0, vertex.getValue().f1, (((VertexWithDegrees)vertex).getInDegree() == vertex.getValue().f0)
&& (((VertexWithDegrees)vertex).getOutDegree() == vertex.getValue().f1) && vertex.getValue().f2));
} catch (Exception e) {
e.printStackTrace();
}
public void updateVertex(Vertex<Long, Long> vertex, MessageIterator<Long> inMessages) {
long inDegree = getInDegree();
long outDegree = getOutDegree();
setNewVertexValue(inDegree + outDegree);
}
}
......@@ -593,16 +670,6 @@ public class VertexCentricConfigurationITCase extends MultipleProgramsTestBase {
}
}
@SuppressWarnings("serial")
public static final class DegreeMessageFunction extends MessagingFunction<Long, Tuple3<Long, Long, Boolean>, Long, Long> {
@Override
public void sendMessages(Vertex<Long, Tuple3<Long, Long, Boolean>> vertex) {
//send message to keep vertices active
sendMessageToAllNeighbors(vertex.getValue().f0);
}
}
@SuppressWarnings("serial")
public static final class AssignOneMapper implements MapFunction<Vertex<Long, Long>, Long> {
......
/*
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.flink.graph.test;
import org.apache.flink.api.java.DataSet;
import org.apache.flink.api.java.ExecutionEnvironment;
import org.apache.flink.api.java.io.DiscardingOutputFormat;
import org.apache.flink.configuration.ConfigConstants;
import org.apache.flink.configuration.Configuration;
import org.apache.flink.graph.Graph;
import org.apache.flink.graph.Vertex;
import org.apache.flink.graph.VertexWithDegrees;
import org.apache.flink.graph.spargel.MessageIterator;
import org.apache.flink.graph.spargel.MessagingFunction;
import org.apache.flink.graph.spargel.VertexUpdateFunction;
import org.apache.flink.test.util.ForkableFlinkMiniCluster;
import org.junit.AfterClass;
import org.junit.BeforeClass;
import org.junit.Test;
import static org.junit.Assert.fail;
public class VertexCentricConfigurationWithExceptionITCase {
private static final int PARALLELISM = 4;
private static ForkableFlinkMiniCluster cluster;
@BeforeClass
public static void setupCluster() {
try {
Configuration config = new Configuration();
config.setInteger(ConfigConstants.TASK_MANAGER_NUM_TASK_SLOTS, PARALLELISM);
cluster = new ForkableFlinkMiniCluster(config, false);
}
catch (Exception e) {
e.printStackTrace();
fail("Error starting test cluster: " + e.getMessage());
}
}
@AfterClass
public static void tearDownCluster() {
try {
cluster.stop();
}
catch (Throwable t) {
t.printStackTrace();
fail("Cluster shutdown caused an exception: " + t.getMessage());
}
}
@Test
public void testOutDegreesNotSet() throws Exception {
/*
* Test that if the degrees are not set, the out degrees cannot be accessed - an
* exception is thrown.
*/
final ExecutionEnvironment env = ExecutionEnvironment.createRemoteEnvironment(
"localhost", cluster.getJobManagerRPCPort());
env.setParallelism(PARALLELISM);
env.getConfig().disableSysoutLogging();
Graph<Long, Long, Long> graph = Graph.fromCollection(TestGraphUtils.getLongLongVertices(),
TestGraphUtils.getLongLongEdges(), env);
try {
DataSet<Vertex<Long, Long>> verticesWithOutDegrees = graph.runVertexCentricIteration(new UpdateFunctionOutDegree(),
new DummyMessageFunction(), 5).getVertices();
verticesWithOutDegrees.output(new DiscardingOutputFormat<Vertex<Long, Long>>());
env.execute();
fail("The degree option not set test did not fail");
} catch (Exception e) {
// We expect the job to fail with an exception
}
}
@Test
public void testInDegreesNotSet() throws Exception {
/*
* Test that if the degrees are not set, the in degrees cannot be accessed - an
* exception is thrown.
*/
final ExecutionEnvironment env = ExecutionEnvironment.createRemoteEnvironment(
"localhost", cluster.getJobManagerRPCPort());
env.setParallelism(PARALLELISM);
env.getConfig().disableSysoutLogging();
Graph<Long, Long, Long> graph = Graph.fromCollection(TestGraphUtils.getLongLongVertices(),
TestGraphUtils.getLongLongEdges(), env);
try {
DataSet<Vertex<Long, Long>> verticesWithInDegrees = graph.runVertexCentricIteration(new UpdateFunctionInDegree(),
new DummyMessageFunction(), 5).getVertices();
verticesWithInDegrees.output(new DiscardingOutputFormat<Vertex<Long, Long>>());
env.execute();
fail("The degree option not set test did not fail");
} catch (Exception e) {
// We expect the job to fail with an exception
}
}
@Test
public void testNumVerticesNotSet() throws Exception {
/*
* Test that if the number of vertices option is not set, this number cannot be accessed -
* an exception id thrown.
*/
final ExecutionEnvironment env = ExecutionEnvironment.createRemoteEnvironment(
"localhost", cluster.getJobManagerRPCPort());
env.setParallelism(PARALLELISM);
env.getConfig().disableSysoutLogging();
Graph<Long, Long, Long> graph = Graph.fromCollection(TestGraphUtils.getLongLongVertices(),
TestGraphUtils.getLongLongEdges(), env);
try {
DataSet<Vertex<Long, Long>> verticesWithNumVertices = graph.runVertexCentricIteration(new UpdateFunctionNumVertices(),
new DummyMessageFunction(), 5).getVertices();
verticesWithNumVertices.output(new DiscardingOutputFormat<Vertex<Long, Long>>());
env.execute();
fail("The num vertices option not set test did not fail");
} catch (Exception e) {
// We expect the job to fail with an exception
}
}
@SuppressWarnings("serial")
public static final class UpdateFunctionInDegree extends VertexUpdateFunction<Long, Long, Long> {
@Override
public void updateVertex(Vertex<Long, Long> vertex, MessageIterator<Long> inMessages) throws Exception {
setNewVertexValue(((VertexWithDegrees) vertex).getInDegree());
}
}
@SuppressWarnings("serial")
public static final class UpdateFunctionOutDegree extends VertexUpdateFunction<Long, Long, Long> {
@Override
public void updateVertex(Vertex<Long, Long> vertex, MessageIterator<Long> inMessages) throws Exception {
setNewVertexValue(((VertexWithDegrees)vertex).getOutDegree());
}
}
@SuppressWarnings("serial")
public static final class UpdateFunctionNumVertices extends VertexUpdateFunction<Long, Long, Long> {
@Override
public void updateVertex(Vertex<Long, Long> vertex, MessageIterator<Long> inMessages) throws Exception {
setNewVertexValue(getNumberOfVertices());
}
}
@SuppressWarnings("serial")
public static final class DummyMessageFunction extends MessagingFunction<Long, Long, Long, Long> {
@Override
public void sendMessages(Vertex<Long, Long> vertex) {
//send message to keep vertices active
sendMessageToAllNeighbors(vertex.getValue());
}
}
}
......@@ -66,6 +66,7 @@ public class ConnectedComponentsWithRandomisedEdgesITCase extends JavaProgramTes
result.writeAsCsv(resultPath, "\n", " ");
env.execute();
}
/**
* A map function that takes a Long value and creates a 2-tuple out of it:
* <pre>(Long value) -> (value, value)</pre>
......
......@@ -28,7 +28,7 @@ import org.apache.flink.graph.Graph;
import org.apache.flink.graph.Vertex;
import org.apache.flink.graph.example.IncrementalSSSPExample;
import org.apache.flink.graph.example.utils.IncrementalSSSPData;
import org.apache.flink.graph.spargel.IterationConfiguration;
import org.apache.flink.graph.spargel.VertexCentricConfiguration;
import org.apache.flink.test.util.MultipleProgramsTestBase;
import org.junit.After;
import org.junit.Before;
......@@ -101,7 +101,7 @@ public class IncrementalSSSPITCase extends MultipleProgramsTestBase {
graph.removeEdge(edgeToBeRemoved);
// configure the iteration
IterationConfiguration parameters = new IterationConfiguration();
VertexCentricConfiguration parameters = new VertexCentricConfiguration();
if(IncrementalSSSPExample.isInSSSP(edgeToBeRemoved, edgesInSSSP)) {
......
Markdown is supported
0% .
You are about to add 0 people to the discussion. Proceed with caution.
先完成此消息的编辑!
想要评论请 注册