K
- The type of the vertex key (the vertex identifier).VV
- The type of the vertex value (the state of the vertex).Message
- The type of the message sent between vertices along the edges.EV
- The type of the values that are associated with the edges.public class VertexCentricIteration<K,VV,EV,Message> extends Object implements CustomUnaryOperation<Vertex<K,VV>,Vertex<K,VV>>
Vertex centric algorithms operate on graphs, which are defined through vertices and edges. The algorithms send messages along the edges and update the state of vertices based on the old state and the incoming messages. All vertices have an initial state. The computation terminates once no vertex receives any message anymore. Additionally, a maximum number of iterations (supersteps) may be specified.
The computation is here represented by one function:
ComputeFunction
receives incoming messages, may update the state for
the vertex, and sends messages along the edges of the vertex.
Vertex-centric graph iterations are run by calling
Graph.runVertexCentricIteration(ComputeFunction, MessageCombiner, int)
.
Modifier and Type | Method and Description |
---|---|
void |
configure(VertexCentricConfiguration parameters)
Configures this vertex-centric iteration with the provided parameters.
|
DataSet<Vertex<K,VV>> |
createResult()
Creates the operator that represents this vertex-centric graph computation.
|
VertexCentricConfiguration |
getIterationConfiguration() |
void |
setInput(DataSet<Vertex<K,VV>> inputData)
Sets the input data set for this operator.
|
static <K,VV,EV,Message> |
withEdges(DataSet<Edge<K,EV>> edgesWithValue,
ComputeFunction<K,VV,EV,Message> cf,
int maximumNumberOfIterations)
Creates a new vertex-centric iteration operator.
|
static <K,VV,EV,Message> |
withEdges(DataSet<Edge<K,EV>> edgesWithValue,
ComputeFunction<K,VV,EV,Message> cf,
MessageCombiner<K,Message> mc,
int maximumNumberOfIterations)
Creates a new vertex-centric iteration operator for graphs where the edges are associated with a value (such as
a weight or distance).
|
public void setInput(DataSet<Vertex<K,VV>> inputData)
setInput
in interface CustomUnaryOperation<Vertex<K,VV>,Vertex<K,VV>>
inputData
- The input data set, which in the case of this operator represents the set of
vertices with their initial state.CustomUnaryOperation.setInput(org.apache.flink.api.java.DataSet)
public DataSet<Vertex<K,VV>> createResult()
The Pregel iteration is mapped to delta iteration as follows. The solution set consists of the set of active vertices and the workset contains the set of messages send to vertices during the previous superstep. Initially, the workset contains a null message for each vertex. In the beginning of a superstep, the solution set is joined with the workset to produce a dataset containing tuples of vertex state and messages (vertex inbox). The superstep compute UDF is realized with a coGroup between the vertices with inbox and the graph edges. The output of the compute UDF contains both the new vertex values and the new messages produced. These are directed to the solution set delta and new workset, respectively, with subsequent flatMaps.
createResult
in interface CustomUnaryOperation<Vertex<K,VV>,Vertex<K,VV>>
public static <K,VV,EV,Message> VertexCentricIteration<K,VV,EV,Message> withEdges(DataSet<Edge<K,EV>> edgesWithValue, ComputeFunction<K,VV,EV,Message> cf, int maximumNumberOfIterations)
K
- The type of the vertex key (the vertex identifier).VV
- The type of the vertex value (the state of the vertex).Message
- The type of the message sent between vertices along the edges.EV
- The type of the values that are associated with the edges.edgesWithValue
- The data set containing edges.cf
- The compute functionpublic static <K,VV,EV,Message> VertexCentricIteration<K,VV,EV,Message> withEdges(DataSet<Edge<K,EV>> edgesWithValue, ComputeFunction<K,VV,EV,Message> cf, MessageCombiner<K,Message> mc, int maximumNumberOfIterations)
K
- The type of the vertex key (the vertex identifier).VV
- The type of the vertex value (the state of the vertex).Message
- The type of the message sent between vertices along the edges.EV
- The type of the values that are associated with the edges.edgesWithValue
- The data set containing edges.cf
- The compute function.mc
- The function that combines messages sent to a vertex during a superstep.public void configure(VertexCentricConfiguration parameters)
parameters
- the configuration parameterspublic VertexCentricConfiguration getIterationConfiguration()
Copyright © 2014–2019 The Apache Software Foundation. All rights reserved.