Modifier | Constructor and Description |
---|---|
protected |
Graph(DataSet<Vertex<K,VV>> vertices,
DataSet<Edge<K,EV>> edges,
ExecutionEnvironment context)
Creates a graph from two DataSets: vertices and edges.
|
Modifier and Type | Method and Description |
---|---|
Graph<K,VV,EV> |
addEdge(Vertex<K,VV> source,
Vertex<K,VV> target,
EV edgeValue)
Adds the given edge to the graph.
|
Graph<K,VV,EV> |
addEdges(List<Edge<K,EV>> newEdges)
Adds the given list edges to the graph.
|
Graph<K,VV,EV> |
addVertex(Vertex<K,VV> vertex)
Adds the input vertex to the graph.
|
Graph<K,VV,EV> |
addVertices(List<Vertex<K,VV>> verticesToAdd)
Adds the list of vertices, passed as input, to the graph.
|
Graph<K,VV,EV> |
difference(Graph<K,VV,EV> graph)
Performs Difference on the vertex and edge sets of the input graphs
removes common vertices and edges.
|
Graph<K,VV,EV> |
filterOnEdges(FilterFunction<Edge<K,EV>> edgeFilter)
Apply a filtering function to the graph and return a sub-graph that
satisfies the predicates only for the edges.
|
Graph<K,VV,EV> |
filterOnVertices(FilterFunction<Vertex<K,VV>> vertexFilter)
Apply a filtering function to the graph and return a sub-graph that
satisfies the predicates only for the vertices.
|
static <K,EV> Graph<K,NullValue,EV> |
fromCollection(Collection<Edge<K,EV>> edges,
ExecutionEnvironment context)
Creates a graph from a Collection of edges.
|
static <K,VV,EV> Graph<K,VV,EV> |
fromCollection(Collection<Edge<K,EV>> edges,
MapFunction<K,VV> vertexValueInitializer,
ExecutionEnvironment context)
Creates a graph from a Collection of edges.
|
static <K,VV,EV> Graph<K,VV,EV> |
fromCollection(Collection<Vertex<K,VV>> vertices,
Collection<Edge<K,EV>> edges,
ExecutionEnvironment context)
Creates a graph from a Collection of vertices and a Collection of edges.
|
static GraphCsvReader |
fromCsvReader(String edgesPath,
ExecutionEnvironment context)
Creates a graph from a CSV file of edges.
|
static <K,VV> GraphCsvReader |
fromCsvReader(String edgesPath,
MapFunction<K,VV> vertexValueInitializer,
ExecutionEnvironment context)
Creates a graph from a CSV file of edges.
|
static GraphCsvReader |
fromCsvReader(String verticesPath,
String edgesPath,
ExecutionEnvironment context)
Creates a Graph from a CSV file of vertices and a CSV file of edges.
|
static <K,EV> Graph<K,NullValue,EV> |
fromDataSet(DataSet<Edge<K,EV>> edges,
ExecutionEnvironment context)
Creates a graph from a DataSet of edges.
|
static <K,VV,EV> Graph<K,VV,EV> |
fromDataSet(DataSet<Edge<K,EV>> edges,
MapFunction<K,VV> vertexValueInitializer,
ExecutionEnvironment context)
Creates a graph from a DataSet of edges.
|
static <K,VV,EV> Graph<K,VV,EV> |
fromDataSet(DataSet<Vertex<K,VV>> vertices,
DataSet<Edge<K,EV>> edges,
ExecutionEnvironment context)
Creates a graph from a DataSet of vertices and a DataSet of edges.
|
static <K> Graph<K,NullValue,NullValue> |
fromTuple2DataSet(DataSet<Tuple2<K,K>> edges,
ExecutionEnvironment context)
Creates a graph from a DataSet of Tuple2 objects for edges.
|
static <K,VV> Graph<K,VV,NullValue> |
fromTuple2DataSet(DataSet<Tuple2<K,K>> edges,
MapFunction<K,VV> vertexValueInitializer,
ExecutionEnvironment context)
Creates a graph from a DataSet of Tuple2 objects for edges.
|
static <K,VV,EV> Graph<K,VV,EV> |
fromTupleDataSet(DataSet<Tuple2<K,VV>> vertices,
DataSet<Tuple3<K,K,EV>> edges,
ExecutionEnvironment context)
Creates a graph from a DataSet of Tuple2 objects for vertices and
Tuple3 objects for edges.
|
static <K,EV> Graph<K,NullValue,EV> |
fromTupleDataSet(DataSet<Tuple3<K,K,EV>> edges,
ExecutionEnvironment context)
Creates a graph from a DataSet of Tuple3 objects for edges.
|
static <K,VV,EV> Graph<K,VV,EV> |
fromTupleDataSet(DataSet<Tuple3<K,K,EV>> edges,
MapFunction<K,VV> vertexValueInitializer,
ExecutionEnvironment context)
Creates a graph from a DataSet of Tuple3 objects for edges.
|
ExecutionEnvironment |
getContext() |
DataSet<Tuple2<K,LongValue>> |
getDegrees()
Return the degree of all vertices in the graph.
|
DataSet<Tuple2<K,K>> |
getEdgeIds() |
DataSet<Edge<K,EV>> |
getEdges() |
DataSet<Tuple3<K,K,EV>> |
getEdgesAsTuple3() |
DataSet<Triplet<K,VV,EV>> |
getTriplets()
This method allows access to the graph's edge values along with its source and target vertex values.
|
Graph<K,VV,EV> |
getUndirected()
This operation adds all inverse-direction edges to the graph.
|
DataSet<K> |
getVertexIds() |
DataSet<Vertex<K,VV>> |
getVertices() |
DataSet<Tuple2<K,VV>> |
getVerticesAsTuple2() |
<T> DataSet<T> |
groupReduceOnEdges(EdgesFunction<K,EV,T> edgesFunction,
EdgeDirection direction)
Groups by vertex and computes a GroupReduce transformation over the edge values of each vertex.
|
<T> DataSet<T> |
groupReduceOnEdges(EdgesFunction<K,EV,T> edgesFunction,
EdgeDirection direction,
TypeInformation<T> typeInfo)
Groups by vertex and computes a GroupReduce transformation over the edge values of each vertex.
|
<T> DataSet<T> |
groupReduceOnEdges(EdgesFunctionWithVertexValue<K,VV,EV,T> edgesFunction,
EdgeDirection direction)
Groups by vertex and computes a GroupReduce transformation over the edge values of each vertex.
|
<T> DataSet<T> |
groupReduceOnEdges(EdgesFunctionWithVertexValue<K,VV,EV,T> edgesFunction,
EdgeDirection direction,
TypeInformation<T> typeInfo)
Groups by vertex and computes a GroupReduce transformation over the edge values of each vertex.
|
<T> DataSet<T> |
groupReduceOnNeighbors(NeighborsFunction<K,VV,EV,T> neighborsFunction,
EdgeDirection direction)
Groups by vertex and computes a GroupReduce transformation over the neighbors (both edges and vertices)
of each vertex.
|
<T> DataSet<T> |
groupReduceOnNeighbors(NeighborsFunction<K,VV,EV,T> neighborsFunction,
EdgeDirection direction,
TypeInformation<T> typeInfo)
Groups by vertex and computes a GroupReduce transformation over the neighbors (both edges and vertices)
of each vertex.
|
<T> DataSet<T> |
groupReduceOnNeighbors(NeighborsFunctionWithVertexValue<K,VV,EV,T> neighborsFunction,
EdgeDirection direction)
Groups by vertex and computes a GroupReduce transformation over the neighbors (both edges and vertices)
of each vertex.
|
<T> DataSet<T> |
groupReduceOnNeighbors(NeighborsFunctionWithVertexValue<K,VV,EV,T> neighborsFunction,
EdgeDirection direction,
TypeInformation<T> typeInfo)
Groups by vertex and computes a GroupReduce transformation over the neighbors (both edges and vertices)
of each vertex.
|
DataSet<Tuple2<K,LongValue>> |
inDegrees()
Return the in-degree of all vertices in the graph.
|
Graph<K,NullValue,EV> |
intersect(Graph<K,VV,EV> graph,
boolean distinctEdges)
Performs intersect on the edge sets of the input graphs.
|
<T> Graph<K,VV,EV> |
joinWithEdges(DataSet<Tuple3<K,K,T>> inputDataSet,
EdgeJoinFunction<EV,T> edgeJoinFunction)
Joins the edge DataSet with an input DataSet on the composite key of both
source and target IDs and applies a user-defined transformation on the values
of the matched records.
|
<T> Graph<K,VV,EV> |
joinWithEdgesOnSource(DataSet<Tuple2<K,T>> inputDataSet,
EdgeJoinFunction<EV,T> edgeJoinFunction)
Joins the edge DataSet with an input Tuple2 DataSet and applies a user-defined transformation
on the values of the matched records.
|
<T> Graph<K,VV,EV> |
joinWithEdgesOnTarget(DataSet<Tuple2<K,T>> inputDataSet,
EdgeJoinFunction<EV,T> edgeJoinFunction)
Joins the edge DataSet with an input Tuple2 DataSet and applies a user-defined transformation
on the values of the matched records.
|
<T> Graph<K,VV,EV> |
joinWithVertices(DataSet<Tuple2<K,T>> inputDataSet,
VertexJoinFunction<VV,T> vertexJoinFunction)
Joins the vertex DataSet of this graph with an input Tuple2 DataSet and applies
a user-defined transformation on the values of the matched records.
|
<NV> Graph<K,VV,NV> |
mapEdges(MapFunction<Edge<K,EV>,NV> mapper)
Apply a function to the attribute of each edge in the graph.
|
<NV> Graph<K,VV,NV> |
mapEdges(MapFunction<Edge<K,EV>,NV> mapper,
TypeInformation<Edge<K,NV>> returnType)
Apply a function to the attribute of each edge in the graph.
|
<NV> Graph<K,NV,EV> |
mapVertices(MapFunction<Vertex<K,VV>,NV> mapper)
Apply a function to the attribute of each vertex in the graph.
|
<NV> Graph<K,NV,EV> |
mapVertices(MapFunction<Vertex<K,VV>,NV> mapper,
TypeInformation<Vertex<K,NV>> returnType)
Apply a function to the attribute of each vertex in the graph.
|
long |
numberOfEdges() |
long |
numberOfVertices() |
DataSet<Tuple2<K,LongValue>> |
outDegrees()
Return the out-degree of all vertices in the graph.
|
DataSet<Tuple2<K,EV>> |
reduceOnEdges(ReduceEdgesFunction<EV> reduceEdgesFunction,
EdgeDirection direction)
Compute a reduce transformation over the edge values of each vertex.
|
DataSet<Tuple2<K,VV>> |
reduceOnNeighbors(ReduceNeighborsFunction<VV> reduceNeighborsFunction,
EdgeDirection direction)
Compute a reduce transformation over the neighbors' vertex values of each vertex.
|
Graph<K,VV,EV> |
removeEdge(Edge<K,EV> edge)
Removes all edges that match the given edge from the graph.
|
Graph<K,VV,EV> |
removeEdges(List<Edge<K,EV>> edgesToBeRemoved)
Removes all the edges that match the edges in the given data set from the graph.
|
Graph<K,VV,EV> |
removeVertex(Vertex<K,VV> vertex)
Removes the given vertex and its edges from the graph.
|
Graph<K,VV,EV> |
removeVertices(List<Vertex<K,VV>> verticesToBeRemoved)
Removes the given list of vertices and its edges from the graph.
|
Graph<K,VV,EV> |
reverse()
Reverse the direction of the edges in the graph.
|
<T> T |
run(GraphAlgorithm<K,VV,EV,T> algorithm) |
<T> GraphAnalytic<K,VV,EV,T> |
run(GraphAnalytic<K,VV,EV,T> analytic)
A
GraphAnalytic is similar to a GraphAlgorithm but is terminal
and results are retrieved via accumulators. |
<M> Graph<K,VV,EV> |
runGatherSumApplyIteration(GatherFunction<VV,EV,M> gatherFunction,
SumFunction<VV,EV,M> sumFunction,
ApplyFunction<K,VV,M> applyFunction,
int maximumNumberOfIterations)
Runs a Gather-Sum-Apply iteration on the graph.
|
<M> Graph<K,VV,EV> |
runGatherSumApplyIteration(GatherFunction<VV,EV,M> gatherFunction,
SumFunction<VV,EV,M> sumFunction,
ApplyFunction<K,VV,M> applyFunction,
int maximumNumberOfIterations,
GSAConfiguration parameters)
Runs a Gather-Sum-Apply iteration on the graph with configuration options.
|
<M> Graph<K,VV,EV> |
runScatterGatherIteration(ScatterFunction<K,VV,M,EV> scatterFunction,
GatherFunction<K,VV,M> gatherFunction,
int maximumNumberOfIterations)
Runs a ScatterGather iteration on the graph.
|
<M> Graph<K,VV,EV> |
runScatterGatherIteration(ScatterFunction<K,VV,M,EV> scatterFunction,
GatherFunction<K,VV,M> gatherFunction,
int maximumNumberOfIterations,
ScatterGatherConfiguration parameters)
Runs a ScatterGather iteration on the graph with configuration options.
|
<M> Graph<K,VV,EV> |
runVertexCentricIteration(ComputeFunction<K,VV,EV,M> computeFunction,
MessageCombiner<K,M> combiner,
int maximumNumberOfIterations)
Runs a
VertexCentricIteration on the graph. |
<M> Graph<K,VV,EV> |
runVertexCentricIteration(ComputeFunction<K,VV,EV,M> computeFunction,
MessageCombiner<K,M> combiner,
int maximumNumberOfIterations,
VertexCentricConfiguration parameters)
Runs a
VertexCentricIteration on the graph with configuration options. |
Graph<K,VV,EV> |
subgraph(FilterFunction<Vertex<K,VV>> vertexFilter,
FilterFunction<Edge<K,EV>> edgeFilter)
Apply filtering functions to the graph and return a sub-graph that
satisfies the predicates for both vertices and edges.
|
<NEW> Graph<K,VV,NEW> |
translateEdgeValues(TranslateFunction<EV,NEW> translator)
Translate
Edge values using the given MapFunction . |
<NEW> Graph<NEW,VV,EV> |
translateGraphIds(TranslateFunction<K,NEW> translator)
|
<NEW> Graph<K,NEW,EV> |
translateVertexValues(TranslateFunction<VV,NEW> translator)
Translate
Vertex values using the given MapFunction . |
Graph<K,VV,EV> |
union(Graph<K,VV,EV> graph)
Performs union on the vertices and edges sets of the input graphs
removing duplicate vertices but maintaining duplicate edges.
|
Boolean |
validate(GraphValidator<K,VV,EV> validator)
Function that checks whether a Graph is a valid Graph,
as defined by the given
GraphValidator . |
public static <K,VV,EV> Graph<K,VV,EV> fromCollection(Collection<Vertex<K,VV>> vertices, Collection<Edge<K,EV>> edges, ExecutionEnvironment context)
vertices
- a Collection of vertices.edges
- a Collection of edges.context
- the flink execution environment.public static <K,EV> Graph<K,NullValue,EV> fromCollection(Collection<Edge<K,EV>> edges, ExecutionEnvironment context)
edges
- a Collection of edges.context
- the flink execution environment.public static <K,VV,EV> Graph<K,VV,EV> fromCollection(Collection<Edge<K,EV>> edges, MapFunction<K,VV> vertexValueInitializer, ExecutionEnvironment context)
edges
- a Collection of edges.vertexValueInitializer
- a map function that initializes the vertex values.
It allows to apply a map transformation on the vertex ID to produce an initial vertex value.context
- the flink execution environment.public static <K,VV,EV> Graph<K,VV,EV> fromDataSet(DataSet<Vertex<K,VV>> vertices, DataSet<Edge<K,EV>> edges, ExecutionEnvironment context)
vertices
- a DataSet of vertices.edges
- a DataSet of edges.context
- the flink execution environment.public static <K,EV> Graph<K,NullValue,EV> fromDataSet(DataSet<Edge<K,EV>> edges, ExecutionEnvironment context)
edges
- a DataSet of edges.context
- the flink execution environment.public static <K,VV,EV> Graph<K,VV,EV> fromDataSet(DataSet<Edge<K,EV>> edges, MapFunction<K,VV> vertexValueInitializer, ExecutionEnvironment context)
edges
- a DataSet of edges.vertexValueInitializer
- the mapper function that initializes the vertex values.
It allows to apply a map transformation on the vertex ID to produce an initial vertex value.context
- the flink execution environment.public static <K,VV,EV> Graph<K,VV,EV> fromTupleDataSet(DataSet<Tuple2<K,VV>> vertices, DataSet<Tuple3<K,K,EV>> edges, ExecutionEnvironment context)
The first field of the Tuple2 vertex object will become the vertex ID and the second field will become the vertex value. The first field of the Tuple3 object for edges will become the source ID, the second field will become the target ID, and the third field will become the edge value.
vertices
- a DataSet of Tuple2 representing the vertices.edges
- a DataSet of Tuple3 representing the edges.context
- the flink execution environment.public static <K,EV> Graph<K,NullValue,EV> fromTupleDataSet(DataSet<Tuple3<K,K,EV>> edges, ExecutionEnvironment context)
The first field of the Tuple3 object will become the source ID, the second field will become the target ID, and the third field will become the edge value.
Vertices are created automatically and their values are set to NullValue.
edges
- a DataSet of Tuple3 representing the edges.context
- the flink execution environment.public static <K,VV,EV> Graph<K,VV,EV> fromTupleDataSet(DataSet<Tuple3<K,K,EV>> edges, MapFunction<K,VV> vertexValueInitializer, ExecutionEnvironment context)
Each Tuple3 will become one Edge, where the source ID will be the first field of the Tuple2, the target ID will be the second field of the Tuple2 and the Edge value will be the third field of the Tuple3.
Vertices are created automatically and their values are initialized by applying the provided vertexValueInitializer map function to the vertex IDs.
edges
- a DataSet of Tuple3.vertexValueInitializer
- the mapper function that initializes the vertex values.
It allows to apply a map transformation on the vertex ID to produce an initial vertex value.context
- the flink execution environment.public static <K> Graph<K,NullValue,NullValue> fromTuple2DataSet(DataSet<Tuple2<K,K>> edges, ExecutionEnvironment context)
Edge value types and Vertex values types will be set to NullValue.
edges
- a DataSet of Tuple2.context
- the flink execution environment.public static <K,VV> Graph<K,VV,NullValue> fromTuple2DataSet(DataSet<Tuple2<K,K>> edges, MapFunction<K,VV> vertexValueInitializer, ExecutionEnvironment context)
Edge value types will be set to NullValue. Vertex values can be initialized by applying a user-defined map function on the vertex IDs.
edges
- a DataSet of Tuple2, where the first field corresponds to the source ID
and the second field corresponds to the target ID.vertexValueInitializer
- the mapper function that initializes the vertex values.
It allows to apply a map transformation on the vertex ID to produce an initial vertex value.context
- the flink execution environment.public static GraphCsvReader fromCsvReader(String verticesPath, String edgesPath, ExecutionEnvironment context)
verticesPath
- path to a CSV file with the Vertex data.edgesPath
- path to a CSV file with the Edge datacontext
- the Flink execution environment.GraphCsvReader
,
on which calling methods to specify types of the Vertex ID, Vertex value and Edge value returns a Graph.GraphCsvReader.types(Class, Class, Class)
,
GraphCsvReader.vertexTypes(Class, Class)
,
GraphCsvReader.edgeTypes(Class, Class)
,
GraphCsvReader.keyType(Class)
public static GraphCsvReader fromCsvReader(String edgesPath, ExecutionEnvironment context)
edgesPath
- a path to a CSV file with the Edges datacontext
- the execution environment.GraphCsvReader
,
on which calling methods to specify types of the Vertex ID, Vertex value and Edge value returns a Graph.GraphCsvReader.types(Class, Class, Class)
,
GraphCsvReader.vertexTypes(Class, Class)
,
GraphCsvReader.edgeTypes(Class, Class)
,
GraphCsvReader.keyType(Class)
public static <K,VV> GraphCsvReader fromCsvReader(String edgesPath, MapFunction<K,VV> vertexValueInitializer, ExecutionEnvironment context)
edgesPath
- a path to a CSV file with the Edge datavertexValueInitializer
- the mapper function that initializes the vertex values.
It allows to apply a map transformation on the vertex ID to produce an initial vertex value.context
- the execution environment.GraphCsvReader
,
on which calling methods to specify types of the Vertex ID, Vertex Value and Edge value returns a Graph.GraphCsvReader.types(Class, Class, Class)
,
GraphCsvReader.vertexTypes(Class, Class)
,
GraphCsvReader.edgeTypes(Class, Class)
,
GraphCsvReader.keyType(Class)
public ExecutionEnvironment getContext()
public Boolean validate(GraphValidator<K,VV,EV> validator) throws Exception
GraphValidator
.Exception
public DataSet<Tuple2<K,VV>> getVerticesAsTuple2()
public DataSet<Tuple3<K,K,EV>> getEdgesAsTuple3()
public DataSet<Triplet<K,VV,EV>> getTriplets()
public <NV> Graph<K,NV,EV> mapVertices(MapFunction<Vertex<K,VV>,NV> mapper)
mapper
- the map function to apply.public <NV> Graph<K,NV,EV> mapVertices(MapFunction<Vertex<K,VV>,NV> mapper, TypeInformation<Vertex<K,NV>> returnType)
mapper
- the map function to apply.returnType
- the explicit return type.public <NV> Graph<K,VV,NV> mapEdges(MapFunction<Edge<K,EV>,NV> mapper)
mapper
- the map function to apply.public <NV> Graph<K,VV,NV> mapEdges(MapFunction<Edge<K,EV>,NV> mapper, TypeInformation<Edge<K,NV>> returnType)
mapper
- the map function to apply.returnType
- the explicit return type.public <NEW> Graph<NEW,VV,EV> translateGraphIds(TranslateFunction<K,NEW> translator) throws Exception
NEW
- new ID typetranslator
- implements conversion from K
to NEW
Exception
public <NEW> Graph<K,NEW,EV> translateVertexValues(TranslateFunction<VV,NEW> translator) throws Exception
Vertex
values using the given MapFunction
.NEW
- new vertex value typetranslator
- implements conversion from VV
to NEW
Exception
public <NEW> Graph<K,VV,NEW> translateEdgeValues(TranslateFunction<EV,NEW> translator) throws Exception
Edge
values using the given MapFunction
.NEW
- new edge value typetranslator
- implements conversion from EV
to NEW
Exception
public <T> Graph<K,VV,EV> joinWithVertices(DataSet<Tuple2<K,T>> inputDataSet, VertexJoinFunction<VV,T> vertexJoinFunction)
T
- the type of the second field of the input Tuple2 DataSet.inputDataSet
- the Tuple2 DataSet to join with.
The first field of the Tuple2 is used as the join key and the second field is passed
as a parameter to the transformation function.vertexJoinFunction
- the transformation function to apply.
The first parameter is the current vertex value and the second parameter is the value
of the matched Tuple2 from the input DataSet.public <T> Graph<K,VV,EV> joinWithEdges(DataSet<Tuple3<K,K,T>> inputDataSet, EdgeJoinFunction<EV,T> edgeJoinFunction)
T
- the type of the third field of the input Tuple3 DataSet.inputDataSet
- the DataSet to join with.
The first two fields of the Tuple3 are used as the composite join key
and the third field is passed as a parameter to the transformation function.edgeJoinFunction
- the transformation function to apply.
The first parameter is the current edge value and the second parameter is the value
of the matched Tuple3 from the input DataSet.public <T> Graph<K,VV,EV> joinWithEdgesOnSource(DataSet<Tuple2<K,T>> inputDataSet, EdgeJoinFunction<EV,T> edgeJoinFunction)
T
- the type of the second field of the input Tuple2 DataSet.inputDataSet
- the DataSet to join with.
The first field of the Tuple2 is used as the join key
and the second field is passed as a parameter to the transformation function.edgeJoinFunction
- the transformation function to apply.
The first parameter is the current edge value and the second parameter is the value
of the matched Tuple2 from the input DataSet.public <T> Graph<K,VV,EV> joinWithEdgesOnTarget(DataSet<Tuple2<K,T>> inputDataSet, EdgeJoinFunction<EV,T> edgeJoinFunction)
T
- the type of the second field of the input Tuple2 DataSet.inputDataSet
- the DataSet to join with.
The first field of the Tuple2 is used as the join key
and the second field is passed as a parameter to the transformation function.edgeJoinFunction
- the transformation function to apply.
The first parameter is the current edge value and the second parameter is the value
of the matched Tuple2 from the input DataSet.public Graph<K,VV,EV> subgraph(FilterFunction<Vertex<K,VV>> vertexFilter, FilterFunction<Edge<K,EV>> edgeFilter)
vertexFilter
- the filter function for vertices.edgeFilter
- the filter function for edges.public Graph<K,VV,EV> filterOnVertices(FilterFunction<Vertex<K,VV>> vertexFilter)
vertexFilter
- the filter function for vertices.public Graph<K,VV,EV> filterOnEdges(FilterFunction<Edge<K,EV>> edgeFilter)
edgeFilter
- the filter function for edges.public DataSet<Tuple2<K,LongValue>> outDegrees()
Tuple2<vertexId, outDegree>
public DataSet<Tuple2<K,LongValue>> inDegrees()
Tuple2<vertexId, inDegree>
public DataSet<Tuple2<K,LongValue>> getDegrees()
Tuple2<vertexId, degree>
public Graph<K,VV,EV> getUndirected()
public <T> DataSet<T> groupReduceOnEdges(EdgesFunctionWithVertexValue<K,VV,EV,T> edgesFunction, EdgeDirection direction) throws IllegalArgumentException
For each vertex, the edgesFunction can iterate over all edges of this vertex with the specified direction, and emit any number of output elements, including none.
T
- the output typeedgesFunction
- the group reduce function to apply to the neighboring edges of each vertex.direction
- the edge direction (in-, out-, all-).IllegalArgumentException
public <T> DataSet<T> groupReduceOnEdges(EdgesFunctionWithVertexValue<K,VV,EV,T> edgesFunction, EdgeDirection direction, TypeInformation<T> typeInfo) throws IllegalArgumentException
For each vertex, the edgesFunction can iterate over all edges of this vertex with the specified direction, and emit any number of output elements, including none.
T
- the output typeedgesFunction
- the group reduce function to apply to the neighboring edges of each vertex.direction
- the edge direction (in-, out-, all-).typeInfo
- the explicit return type.IllegalArgumentException
public <T> DataSet<T> groupReduceOnEdges(EdgesFunction<K,EV,T> edgesFunction, EdgeDirection direction) throws IllegalArgumentException
For each vertex, the edgesFunction can iterate over all edges of this vertex with the specified direction, and emit any number of output elements, including none.
T
- the output typeedgesFunction
- the group reduce function to apply to the neighboring edges of each vertex.direction
- the edge direction (in-, out-, all-).IllegalArgumentException
public <T> DataSet<T> groupReduceOnEdges(EdgesFunction<K,EV,T> edgesFunction, EdgeDirection direction, TypeInformation<T> typeInfo) throws IllegalArgumentException
For each vertex, the edgesFunction can iterate over all edges of this vertex with the specified direction, and emit any number of output elements, including none.
T
- the output typeedgesFunction
- the group reduce function to apply to the neighboring edges of each vertex.direction
- the edge direction (in-, out-, all-).typeInfo
- the explicit return type.IllegalArgumentException
public Graph<K,VV,EV> reverse() throws UnsupportedOperationException
UnsupportedOperationException
public long numberOfVertices() throws Exception
Exception
public long numberOfEdges() throws Exception
Exception
public Graph<K,VV,EV> addVertex(Vertex<K,VV> vertex)
vertex
- the vertex to be addedpublic Graph<K,VV,EV> addVertices(List<Vertex<K,VV>> verticesToAdd)
verticesToAdd
- the list of vertices to addpublic Graph<K,VV,EV> addEdge(Vertex<K,VV> source, Vertex<K,VV> target, EV edgeValue)
source
- the source vertex of the edgetarget
- the target vertex of the edgeedgeValue
- the edge valuepublic Graph<K,VV,EV> addEdges(List<Edge<K,EV>> newEdges)
When adding an edge for a non-existing set of vertices, the edge is considered invalid and ignored.
newEdges
- the data set of edges to be addedpublic Graph<K,VV,EV> removeVertex(Vertex<K,VV> vertex)
vertex
- the vertex to removepublic Graph<K,VV,EV> removeVertices(List<Vertex<K,VV>> verticesToBeRemoved)
verticesToBeRemoved
- the list of vertices to be removedpublic Graph<K,VV,EV> removeEdge(Edge<K,EV> edge)
edge
- the edge to removepublic Graph<K,VV,EV> removeEdges(List<Edge<K,EV>> edgesToBeRemoved)
edgesToBeRemoved
- the list of edges to be removedpublic Graph<K,VV,EV> union(Graph<K,VV,EV> graph)
graph
- the graph to perform union withpublic Graph<K,VV,EV> difference(Graph<K,VV,EV> graph)
graph
- the graph to perform difference withpublic Graph<K,NullValue,EV> intersect(Graph<K,VV,EV> graph, boolean distinctEdges)
The method computes pairs of equal edges from the input graphs. If the same edge occurs
multiple times in the input graphs, there will be multiple edge pairs to be considered. Each
edge instance can only be part of one pair. If the given parameter distinctEdges
is set
to true
, there will be exactly one edge in the output graph representing all pairs of
equal edges. If the parameter is set to false
, both edges of each pair will be in the
output.
Vertices in the output graph will have no vertex values.
graph
- the graph to perform intersect withdistinctEdges
- if set to true
, there will be exactly one edge in the output graph
representing all pairs of equal edges, otherwise, for each pair, both
edges will be in the output graphpublic <M> Graph<K,VV,EV> runScatterGatherIteration(ScatterFunction<K,VV,M,EV> scatterFunction, GatherFunction<K,VV,M> gatherFunction, int maximumNumberOfIterations)
scatterFunction
- the scatter functiongatherFunction
- the gather functionmaximumNumberOfIterations
- maximum number of iterations to performpublic <M> Graph<K,VV,EV> runScatterGatherIteration(ScatterFunction<K,VV,M,EV> scatterFunction, GatherFunction<K,VV,M> gatherFunction, int maximumNumberOfIterations, ScatterGatherConfiguration parameters)
scatterFunction
- the scatter functiongatherFunction
- the gather functionmaximumNumberOfIterations
- maximum number of iterations to performparameters
- the iteration configuration parameterspublic <M> Graph<K,VV,EV> runGatherSumApplyIteration(GatherFunction<VV,EV,M> gatherFunction, SumFunction<VV,EV,M> sumFunction, ApplyFunction<K,VV,M> applyFunction, int maximumNumberOfIterations)
M
- the intermediate type used between gather, sum and applygatherFunction
- the gather function collects information about adjacent vertices and edgessumFunction
- the sum function aggregates the gathered informationapplyFunction
- the apply function updates the vertex values with the aggregatesmaximumNumberOfIterations
- maximum number of iterations to performpublic <M> Graph<K,VV,EV> runGatherSumApplyIteration(GatherFunction<VV,EV,M> gatherFunction, SumFunction<VV,EV,M> sumFunction, ApplyFunction<K,VV,M> applyFunction, int maximumNumberOfIterations, GSAConfiguration parameters)
M
- the intermediate type used between gather, sum and applygatherFunction
- the gather function collects information about adjacent vertices and edgessumFunction
- the sum function aggregates the gathered informationapplyFunction
- the apply function updates the vertex values with the aggregatesmaximumNumberOfIterations
- maximum number of iterations to performparameters
- the iteration configuration parameterspublic <M> Graph<K,VV,EV> runVertexCentricIteration(ComputeFunction<K,VV,EV,M> computeFunction, MessageCombiner<K,M> combiner, int maximumNumberOfIterations)
VertexCentricIteration
on the graph.
No configuration options are provided.computeFunction
- the vertex compute functioncombiner
- an optional message combinermaximumNumberOfIterations
- maximum number of iterations to performpublic <M> Graph<K,VV,EV> runVertexCentricIteration(ComputeFunction<K,VV,EV,M> computeFunction, MessageCombiner<K,M> combiner, int maximumNumberOfIterations, VertexCentricConfiguration parameters)
VertexCentricIteration
on the graph with configuration options.computeFunction
- the vertex compute functioncombiner
- an optional message combinermaximumNumberOfIterations
- maximum number of iterations to performparameters
- the VertexCentricConfiguration
parameterspublic <T> T run(GraphAlgorithm<K,VV,EV,T> algorithm) throws Exception
T
- the return typealgorithm
- the algorithm to run on the GraphException
public <T> GraphAnalytic<K,VV,EV,T> run(GraphAnalytic<K,VV,EV,T> analytic) throws Exception
GraphAnalytic
is similar to a GraphAlgorithm
but is terminal
and results are retrieved via accumulators. A Flink program has a single
point of execution. A GraphAnalytic
defers execution to the user to
allow composing multiple analytics and algorithms into a single program.T
- the result typeanalytic
- the analytic to run on the GraphException
public <T> DataSet<T> groupReduceOnNeighbors(NeighborsFunctionWithVertexValue<K,VV,EV,T> neighborsFunction, EdgeDirection direction) throws IllegalArgumentException
For each vertex, the neighborsFunction can iterate over all neighbors of this vertex with the specified direction, and emit any number of output elements, including none.
T
- the output typeneighborsFunction
- the group reduce function to apply to the neighboring edges and vertices
of each vertex.direction
- the edge direction (in-, out-, all-).IllegalArgumentException
public <T> DataSet<T> groupReduceOnNeighbors(NeighborsFunctionWithVertexValue<K,VV,EV,T> neighborsFunction, EdgeDirection direction, TypeInformation<T> typeInfo) throws IllegalArgumentException
For each vertex, the neighborsFunction can iterate over all neighbors of this vertex with the specified direction, and emit any number of output elements, including none.
T
- the output typeneighborsFunction
- the group reduce function to apply to the neighboring edges and vertices
of each vertex.direction
- the edge direction (in-, out-, all-).typeInfo
- the explicit return typeIllegalArgumentException
public <T> DataSet<T> groupReduceOnNeighbors(NeighborsFunction<K,VV,EV,T> neighborsFunction, EdgeDirection direction) throws IllegalArgumentException
For each vertex, the neighborsFunction can iterate over all neighbors of this vertex with the specified direction, and emit any number of output elements, including none.
T
- the output typeneighborsFunction
- the group reduce function to apply to the neighboring edges and vertices
of each vertex.direction
- the edge direction (in-, out-, all-).IllegalArgumentException
public <T> DataSet<T> groupReduceOnNeighbors(NeighborsFunction<K,VV,EV,T> neighborsFunction, EdgeDirection direction, TypeInformation<T> typeInfo) throws IllegalArgumentException
For each vertex, the neighborsFunction can iterate over all neighbors of this vertex with the specified direction, and emit any number of output elements, including none.
T
- the output typeneighborsFunction
- the group reduce function to apply to the neighboring edges and vertices
of each vertex.direction
- the edge direction (in-, out-, all-).typeInfo
- the explicit return typeIllegalArgumentException
public DataSet<Tuple2<K,VV>> reduceOnNeighbors(ReduceNeighborsFunction<VV> reduceNeighborsFunction, EdgeDirection direction) throws IllegalArgumentException
ReduceNeighborsFunction
until only a single value for each vertex remains.
The ReduceNeighborsFunction
combines a pair of neighbor vertex values
into one new value of the same type.reduceNeighborsFunction
- the reduce function to apply to the neighbors of each vertex.direction
- the edge direction (in-, out-, all-)ReduceNeighborsFunction
.IllegalArgumentException
public DataSet<Tuple2<K,EV>> reduceOnEdges(ReduceEdgesFunction<EV> reduceEdgesFunction, EdgeDirection direction) throws IllegalArgumentException
ReduceEdgesFunction
until only a single value for each edge remains.
The ReduceEdgesFunction
combines two edge values into one new value of the same type.reduceEdgesFunction
- the reduce function to apply to the neighbors of each vertex.direction
- the edge direction (in-, out-, all-)ReduceEdgesFunction
.IllegalArgumentException
Copyright © 2014–2020 The Apache Software Foundation. All rights reserved.