Skip to content
体验新版
项目
组织
正在加载...
登录
切换导航
打开侧边栏
doujutun3207
flink
提交
ce2163e6
F
flink
项目概览
doujutun3207
/
flink
与 Fork 源项目一致
从无法访问的项目Fork
通知
24
Star
0
Fork
0
代码
文件
提交
分支
Tags
贡献者
分支图
Diff
Issue
0
列表
看板
标记
里程碑
合并请求
0
Wiki
0
Wiki
分析
仓库
DevOps
项目成员
Pages
F
flink
项目概览
项目概览
详情
发布
仓库
仓库
文件
提交
分支
标签
贡献者
分支图
比较
Issue
0
Issue
0
列表
看板
标记
里程碑
合并请求
0
合并请求
0
Pages
分析
分析
仓库分析
DevOps
Wiki
0
Wiki
成员
成员
收起侧边栏
关闭侧边栏
动态
分支图
创建新Issue
提交
Issue看板
体验新版 GitCode,发现更多精彩内容 >>
提交
ce2163e6
编写于
5月 07, 2015
作者:
V
vasia
浏览文件
操作
浏览文件
下载
电子邮件补丁
差异文件
[gelly] removes generic type constraints
This closes #657
上级
ddb2b347
变更
29
展开全部
隐藏空白更改
内联
并排
Showing
29 changed file
with
116 addition
and
368 deletion
+116
-368
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/Edge.java
...link-gelly/src/main/java/org/apache/flink/graph/Edge.java
+1
-4
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/EdgesFunction.java
...y/src/main/java/org/apache/flink/graph/EdgesFunction.java
+1
-2
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/EdgesFunctionWithVertexValue.java
.../org/apache/flink/graph/EdgesFunctionWithVertexValue.java
+1
-2
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/Graph.java
...ink-gelly/src/main/java/org/apache/flink/graph/Graph.java
+68
-135
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/GraphAlgorithm.java
.../src/main/java/org/apache/flink/graph/GraphAlgorithm.java
+1
-3
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/NeighborsFunction.java
...c/main/java/org/apache/flink/graph/NeighborsFunction.java
+1
-2
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/NeighborsFunctionWithVertexValue.java
.../apache/flink/graph/NeighborsFunctionWithVertexValue.java
+1
-2
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/ReduceEdgesFunction.java
...main/java/org/apache/flink/graph/ReduceEdgesFunction.java
+1
-1
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/ReduceNeighborsFunction.java
.../java/org/apache/flink/graph/ReduceNeighborsFunction.java
+1
-1
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/Triplet.java
...k-gelly/src/main/java/org/apache/flink/graph/Triplet.java
+1
-4
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/Vertex.java
...nk-gelly/src/main/java/org/apache/flink/graph/Vertex.java
+1
-4
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/gsa/ApplyFunction.java
...c/main/java/org/apache/flink/graph/gsa/ApplyFunction.java
+1
-2
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/gsa/GatherFunction.java
.../main/java/org/apache/flink/graph/gsa/GatherFunction.java
+1
-1
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/gsa/GatherSumApplyIteration.java
...a/org/apache/flink/graph/gsa/GatherSumApplyIteration.java
+9
-16
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/gsa/Neighbor.java
...ly/src/main/java/org/apache/flink/graph/gsa/Neighbor.java
+1
-4
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/gsa/SumFunction.java
...src/main/java/org/apache/flink/graph/gsa/SumFunction.java
+1
-1
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/library/LabelPropagation.java
...java/org/apache/flink/graph/library/LabelPropagation.java
+3
-7
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/library/PageRank.java
...rc/main/java/org/apache/flink/graph/library/PageRank.java
+3
-8
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/library/SingleSourceShortestPaths.java
...apache/flink/graph/library/SingleSourceShortestPaths.java
+4
-10
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/spargel/MessagingFunction.java
...ava/org/apache/flink/graph/spargel/MessagingFunction.java
+2
-4
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/spargel/VertexCentricIteration.java
...rg/apache/flink/graph/spargel/VertexCentricIteration.java
+4
-9
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/spargel/VertexUpdateFunction.java
.../org/apache/flink/graph/spargel/VertexUpdateFunction.java
+1
-2
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/utils/EdgeToTuple3Map.java
...in/java/org/apache/flink/graph/utils/EdgeToTuple3Map.java
+1
-4
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/utils/Tuple2ToVertexMap.java
.../java/org/apache/flink/graph/utils/Tuple2ToVertexMap.java
+1
-4
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/utils/Tuple3ToEdgeMap.java
...in/java/org/apache/flink/graph/utils/Tuple3ToEdgeMap.java
+1
-4
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/utils/VertexToTuple2Map.java
.../java/org/apache/flink/graph/utils/VertexToTuple2Map.java
+1
-4
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/validation/GraphValidator.java
...ava/org/apache/flink/graph/validation/GraphValidator.java
+1
-2
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/validation/InvalidVertexIdsValidator.java
...che/flink/graph/validation/InvalidVertexIdsValidator.java
+3
-8
flink-staging/flink-gelly/src/test/java/org/apache/flink/graph/test/WeaklyConnectedITCase.java
...va/org/apache/flink/graph/test/WeaklyConnectedITCase.java
+0
-118
未找到文件。
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/Edge.java
浏览文件 @
ce2163e6
...
...
@@ -18,8 +18,6 @@
package
org.apache.flink.graph
;
import
java.io.Serializable
;
import
org.apache.flink.api.java.tuple.Tuple3
;
/**
...
...
@@ -30,8 +28,7 @@ import org.apache.flink.api.java.tuple.Tuple3;
* @param <K> the key type for the sources and target vertices
* @param <V> the edge value type
*/
public
class
Edge
<
K
extends
Comparable
<
K
>
&
Serializable
,
V
extends
Serializable
>
extends
Tuple3
<
K
,
K
,
V
>{
public
class
Edge
<
K
,
V
>
extends
Tuple3
<
K
,
K
,
V
>{
private
static
final
long
serialVersionUID
=
1L
;
...
...
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/EdgesFunction.java
浏览文件 @
ce2163e6
...
...
@@ -32,8 +32,7 @@ import org.apache.flink.util.Collector;
* @param <EV> the edge value type
* @param <O> the type of the return value
*/
public
interface
EdgesFunction
<
K
extends
Comparable
<
K
>
&
Serializable
,
EV
extends
Serializable
,
O
>
extends
Function
,
Serializable
{
public
interface
EdgesFunction
<
K
,
EV
,
O
>
extends
Function
,
Serializable
{
void
iterateEdges
(
Iterable
<
Tuple2
<
K
,
Edge
<
K
,
EV
>>>
edges
,
Collector
<
O
>
out
)
throws
Exception
;
}
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/EdgesFunctionWithVertexValue.java
浏览文件 @
ce2163e6
...
...
@@ -33,8 +33,7 @@ import org.apache.flink.util.Collector;
* @param <EV> the edge value type
* @param <O> the type of the return value
*/
public
interface
EdgesFunctionWithVertexValue
<
K
extends
Comparable
<
K
>
&
Serializable
,
VV
extends
Serializable
,
EV
extends
Serializable
,
O
>
extends
Function
,
Serializable
{
public
interface
EdgesFunctionWithVertexValue
<
K
,
VV
,
EV
,
O
>
extends
Function
,
Serializable
{
void
iterateEdges
(
Vertex
<
K
,
VV
>
v
,
Iterable
<
Edge
<
K
,
EV
>>
edges
,
Collector
<
O
>
out
)
throws
Exception
;
}
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/Graph.java
浏览文件 @
ce2163e6
此差异已折叠。
点击以展开。
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/GraphAlgorithm.java
浏览文件 @
ce2163e6
...
...
@@ -18,14 +18,12 @@
package
org.apache.flink.graph
;
import
java.io.Serializable
;
/**
* @param <K> key type
* @param <VV> vertex value type
* @param <EV> edge value type
*/
public
interface
GraphAlgorithm
<
K
extends
Comparable
<
K
>
&
Serializable
,
VV
extends
Serializable
,
EV
extends
Serializable
>
{
public
interface
GraphAlgorithm
<
K
,
VV
,
EV
>
{
public
Graph
<
K
,
VV
,
EV
>
run
(
Graph
<
K
,
VV
,
EV
>
input
)
throws
Exception
;
}
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/NeighborsFunction.java
浏览文件 @
ce2163e6
...
...
@@ -34,8 +34,7 @@ import org.apache.flink.util.Collector;
* @param <EV> the edge value type
* @param <O> the type of the return value
*/
public
interface
NeighborsFunction
<
K
extends
Comparable
<
K
>
&
Serializable
,
VV
extends
Serializable
,
EV
extends
Serializable
,
O
>
extends
Function
,
Serializable
{
public
interface
NeighborsFunction
<
K
,
VV
,
EV
,
O
>
extends
Function
,
Serializable
{
void
iterateNeighbors
(
Iterable
<
Tuple3
<
K
,
Edge
<
K
,
EV
>,
Vertex
<
K
,
VV
>>>
neighbors
,
Collector
<
O
>
out
)
throws
Exception
;
}
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/NeighborsFunctionWithVertexValue.java
浏览文件 @
ce2163e6
...
...
@@ -34,8 +34,7 @@ import org.apache.flink.util.Collector;
* @param <EV> the edge value type
* @param <O> the type of the return value
*/
public
interface
NeighborsFunctionWithVertexValue
<
K
extends
Comparable
<
K
>
&
Serializable
,
VV
extends
Serializable
,
EV
extends
Serializable
,
O
>
extends
Function
,
Serializable
{
public
interface
NeighborsFunctionWithVertexValue
<
K
,
VV
,
EV
,
O
>
extends
Function
,
Serializable
{
void
iterateNeighbors
(
Vertex
<
K
,
VV
>
vertex
,
Iterable
<
Tuple2
<
Edge
<
K
,
EV
>,
Vertex
<
K
,
VV
>>>
neighbors
,
Collector
<
O
>
out
)
throws
Exception
;
}
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/ReduceEdgesFunction.java
浏览文件 @
ce2163e6
...
...
@@ -28,7 +28,7 @@ import java.io.Serializable;
*
* @param <EV> the edge value type
*/
public
interface
ReduceEdgesFunction
<
EV
extends
Serializable
>
extends
Function
,
Serializable
{
public
interface
ReduceEdgesFunction
<
EV
>
extends
Function
,
Serializable
{
EV
reduceEdges
(
EV
firstEdgeValue
,
EV
secondEdgeValue
);
}
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/ReduceNeighborsFunction.java
浏览文件 @
ce2163e6
...
...
@@ -29,7 +29,7 @@ import java.io.Serializable;
*
* @param <VV> the vertex value type
*/
public
interface
ReduceNeighborsFunction
<
VV
extends
Serializable
>
extends
Function
,
Serializable
{
public
interface
ReduceNeighborsFunction
<
VV
>
extends
Function
,
Serializable
{
VV
reduceNeighbors
(
VV
firstNeighborValue
,
VV
secondNeighborValue
);
}
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/Triplet.java
浏览文件 @
ce2163e6
...
...
@@ -20,8 +20,6 @@ package org.apache.flink.graph;
import
org.apache.flink.api.java.tuple.Tuple5
;
import
java.io.Serializable
;
/**
* A Triplet stores and retrieves the edges along with their corresponding source and target vertices.
* Triplets can be obtained from the input graph via the {@link org.apache.flink.graph.Graph#getTriplets()} method.
...
...
@@ -30,8 +28,7 @@ import java.io.Serializable;
* @param <VV> the vertex value type
* @param <EV> the edge value type
*/
public
class
Triplet
<
K
extends
Comparable
<
K
>
&
Serializable
,
VV
extends
Serializable
,
EV
extends
Serializable
>
extends
Tuple5
<
K
,
K
,
VV
,
VV
,
EV
>
{
public
class
Triplet
<
K
,
VV
,
EV
>
extends
Tuple5
<
K
,
K
,
VV
,
VV
,
EV
>
{
private
static
final
long
serialVersionUID
=
1L
;
...
...
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/Vertex.java
浏览文件 @
ce2163e6
...
...
@@ -18,8 +18,6 @@
package
org.apache.flink.graph
;
import
java.io.Serializable
;
import
org.apache.flink.api.java.tuple.Tuple2
;
/**
...
...
@@ -29,8 +27,7 @@ import org.apache.flink.api.java.tuple.Tuple2;
* @param <K>
* @param <V>
*/
public
class
Vertex
<
K
extends
Comparable
<
K
>
&
Serializable
,
V
extends
Serializable
>
extends
Tuple2
<
K
,
V
>
{
public
class
Vertex
<
K
,
V
>
extends
Tuple2
<
K
,
V
>
{
private
static
final
long
serialVersionUID
=
1L
;
...
...
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/gsa/ApplyFunction.java
浏览文件 @
ce2163e6
...
...
@@ -25,8 +25,7 @@ import org.apache.flink.util.Collector;
import
java.io.Serializable
;
@SuppressWarnings
(
"serial"
)
public
abstract
class
ApplyFunction
<
K
extends
Comparable
<
K
>
&
Serializable
,
VV
extends
Serializable
,
M
>
implements
Serializable
{
public
abstract
class
ApplyFunction
<
K
,
VV
,
M
>
implements
Serializable
{
public
abstract
void
apply
(
M
newValue
,
VV
currentValue
);
...
...
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/gsa/GatherFunction.java
浏览文件 @
ce2163e6
...
...
@@ -23,7 +23,7 @@ import org.apache.flink.api.common.functions.IterationRuntimeContext;
import
java.io.Serializable
;
@SuppressWarnings
(
"serial"
)
public
abstract
class
GatherFunction
<
VV
extends
Serializable
,
EV
extends
Serializable
,
M
>
implements
Serializable
{
public
abstract
class
GatherFunction
<
VV
,
EV
,
M
>
implements
Serializable
{
public
abstract
M
gather
(
Neighbor
<
VV
,
EV
>
neighbor
);
...
...
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/gsa/GatherSumApplyIteration.java
浏览文件 @
ce2163e6
...
...
@@ -39,8 +39,6 @@ import org.apache.flink.graph.Edge;
import
org.apache.flink.graph.Vertex
;
import
org.apache.flink.util.Collector
;
import
java.io.Serializable
;
/**
* This class represents iterative graph computations, programmed in a gather-sum-apply perspective.
*
...
...
@@ -49,8 +47,7 @@ import java.io.Serializable;
* @param <EV> The type of the edge value in the graph
* @param <M> The intermediate type used by the gather, sum and apply functions
*/
public
class
GatherSumApplyIteration
<
K
extends
Comparable
<
K
>
&
Serializable
,
VV
extends
Serializable
,
EV
extends
Serializable
,
M
>
implements
CustomUnaryOperation
<
Vertex
<
K
,
VV
>,
public
class
GatherSumApplyIteration
<
K
,
VV
,
EV
,
M
>
implements
CustomUnaryOperation
<
Vertex
<
K
,
VV
>,
Vertex
<
K
,
VV
>>
{
private
DataSet
<
Vertex
<
K
,
VV
>>
vertexDataSet
;
...
...
@@ -159,10 +156,10 @@ public class GatherSumApplyIteration<K extends Comparable<K> & Serializable,
*
* @return An in stance of the gather-sum-apply graph computation operator.
*/
public
static
final
<
K
extends
Comparable
<
K
>
&
Serializable
,
VV
extends
Serializable
,
EV
extends
Serializable
,
M
>
GatherSumApplyIteration
<
K
,
VV
,
EV
,
M
>
withEdges
(
DataSet
<
Edge
<
K
,
EV
>>
edges
,
GatherFunction
<
VV
,
EV
,
M
>
gather
,
SumFunction
<
VV
,
EV
,
M
>
sum
,
ApplyFunction
<
K
,
VV
,
M
>
apply
,
int
maximumNumberOfIterations
)
{
public
static
final
<
K
,
VV
,
EV
,
M
>
GatherSumApplyIteration
<
K
,
VV
,
EV
,
M
>
withEdges
(
DataSet
<
Edge
<
K
,
EV
>>
edges
,
GatherFunction
<
VV
,
EV
,
M
>
gather
,
SumFunction
<
VV
,
EV
,
M
>
sum
,
ApplyFunction
<
K
,
VV
,
M
>
apply
,
int
maximumNumberOfIterations
)
{
return
new
GatherSumApplyIteration
<
K
,
VV
,
EV
,
M
>(
gather
,
sum
,
apply
,
edges
,
maximumNumberOfIterations
);
}
...
...
@@ -172,8 +169,7 @@ public class GatherSumApplyIteration<K extends Comparable<K> & Serializable,
@SuppressWarnings
(
"serial"
)
@ForwardedFields
(
"f0"
)
private
static
final
class
GatherUdf
<
K
extends
Comparable
<
K
>
&
Serializable
,
VV
extends
Serializable
,
EV
extends
Serializable
,
M
>
extends
RichMapFunction
<
Tuple2
<
K
,
Neighbor
<
VV
,
EV
>>,
private
static
final
class
GatherUdf
<
K
,
VV
,
EV
,
M
>
extends
RichMapFunction
<
Tuple2
<
K
,
Neighbor
<
VV
,
EV
>>,
Tuple2
<
K
,
M
>>
implements
ResultTypeQueryable
<
Tuple2
<
K
,
M
>>
{
private
final
GatherFunction
<
VV
,
EV
,
M
>
gatherFunction
;
...
...
@@ -210,8 +206,7 @@ public class GatherSumApplyIteration<K extends Comparable<K> & Serializable,
}
@SuppressWarnings
(
"serial"
)
private
static
final
class
SumUdf
<
K
extends
Comparable
<
K
>
&
Serializable
,
VV
extends
Serializable
,
EV
extends
Serializable
,
M
>
extends
RichReduceFunction
<
Tuple2
<
K
,
M
>>
private
static
final
class
SumUdf
<
K
,
VV
,
EV
,
M
>
extends
RichReduceFunction
<
Tuple2
<
K
,
M
>>
implements
ResultTypeQueryable
<
Tuple2
<
K
,
M
>>{
private
final
SumFunction
<
VV
,
EV
,
M
>
sumFunction
;
...
...
@@ -249,8 +244,7 @@ public class GatherSumApplyIteration<K extends Comparable<K> & Serializable,
}
@SuppressWarnings
(
"serial"
)
private
static
final
class
ApplyUdf
<
K
extends
Comparable
<
K
>
&
Serializable
,
VV
extends
Serializable
,
EV
extends
Serializable
,
M
>
extends
RichFlatJoinFunction
<
Tuple2
<
K
,
M
>,
private
static
final
class
ApplyUdf
<
K
,
VV
,
EV
,
M
>
extends
RichFlatJoinFunction
<
Tuple2
<
K
,
M
>,
Vertex
<
K
,
VV
>,
Vertex
<
K
,
VV
>>
implements
ResultTypeQueryable
<
Vertex
<
K
,
VV
>>
{
private
final
ApplyFunction
<
K
,
VV
,
M
>
applyFunction
;
...
...
@@ -289,8 +283,7 @@ public class GatherSumApplyIteration<K extends Comparable<K> & Serializable,
@SuppressWarnings
(
"serial"
)
@ForwardedFieldsSecond
(
"f1->f0"
)
private
static
final
class
ProjectKeyWithNeighbor
<
K
extends
Comparable
<
K
>
&
Serializable
,
VV
extends
Serializable
,
EV
extends
Serializable
>
implements
FlatJoinFunction
<
private
static
final
class
ProjectKeyWithNeighbor
<
K
,
VV
,
EV
>
implements
FlatJoinFunction
<
Vertex
<
K
,
VV
>,
Edge
<
K
,
EV
>,
Tuple2
<
K
,
Neighbor
<
VV
,
EV
>>>
{
public
void
join
(
Vertex
<
K
,
VV
>
vertex
,
Edge
<
K
,
EV
>
edge
,
Collector
<
Tuple2
<
K
,
Neighbor
<
VV
,
EV
>>>
out
)
{
...
...
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/gsa/Neighbor.java
浏览文件 @
ce2163e6
...
...
@@ -20,8 +20,6 @@ package org.apache.flink.graph.gsa;
import
org.apache.flink.api.java.tuple.Tuple2
;
import
java.io.Serializable
;
/**
* This class represents a <sourceVertex, edge> pair
* This is a wrapper around Tuple2<VV, EV> for convenience in the GatherFunction
...
...
@@ -29,8 +27,7 @@ import java.io.Serializable;
* @param <EV> the edge value type
*/
@SuppressWarnings
(
"serial"
)
public
class
Neighbor
<
VV
extends
Serializable
,
EV
extends
Serializable
>
extends
Tuple2
<
VV
,
EV
>
{
public
class
Neighbor
<
VV
,
EV
>
extends
Tuple2
<
VV
,
EV
>
{
public
Neighbor
()
{}
...
...
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/gsa/SumFunction.java
浏览文件 @
ce2163e6
...
...
@@ -23,7 +23,7 @@ import org.apache.flink.api.common.functions.IterationRuntimeContext;
import
java.io.Serializable
;
@SuppressWarnings
(
"serial"
)
public
abstract
class
SumFunction
<
VV
extends
Serializable
,
EV
extends
Serializable
,
M
>
implements
Serializable
{
public
abstract
class
SumFunction
<
VV
,
EV
,
M
>
implements
Serializable
{
public
abstract
M
sum
(
M
arg0
,
M
arg1
);
...
...
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/library/LabelPropagation.java
浏览文件 @
ce2163e6
...
...
@@ -25,7 +25,6 @@ import org.apache.flink.graph.spargel.MessagingFunction;
import
org.apache.flink.graph.spargel.VertexUpdateFunction
;
import
org.apache.flink.types.NullValue
;
import
java.io.Serializable
;
import
java.util.HashMap
;
import
java.util.Map
;
import
java.util.Map.Entry
;
...
...
@@ -41,8 +40,7 @@ import java.util.Map.Entry;
*
*/
@SuppressWarnings
(
"serial"
)
public
class
LabelPropagation
<
K
extends
Comparable
<
K
>
&
Serializable
>
implements
GraphAlgorithm
<
K
,
Long
,
NullValue
>
{
public
class
LabelPropagation
<
K
>
implements
GraphAlgorithm
<
K
,
Long
,
NullValue
>
{
private
final
int
maxIterations
;
...
...
@@ -63,8 +61,7 @@ public class LabelPropagation<K extends Comparable<K> & Serializable>
* Function that updates the value of a vertex by adopting the most frequent
* label among its in-neighbors
*/
public
static
final
class
UpdateVertexLabel
<
K
extends
Comparable
<
K
>
&
Serializable
>
extends
VertexUpdateFunction
<
K
,
Long
,
Long
>
{
public
static
final
class
UpdateVertexLabel
<
K
>
extends
VertexUpdateFunction
<
K
,
Long
,
Long
>
{
public
void
updateVertex
(
K
vertexKey
,
Long
vertexValue
,
MessageIterator
<
Long
>
inMessages
)
{
...
...
@@ -105,8 +102,7 @@ public class LabelPropagation<K extends Comparable<K> & Serializable>
/**
* Sends the vertex label to all out-neighbors
*/
public
static
final
class
SendNewLabelToNeighbors
<
K
extends
Comparable
<
K
>
&
Serializable
>
extends
MessagingFunction
<
K
,
Long
,
Long
,
NullValue
>
{
public
static
final
class
SendNewLabelToNeighbors
<
K
>
extends
MessagingFunction
<
K
,
Long
,
Long
,
NullValue
>
{
public
void
sendMessages
(
K
vertexKey
,
Long
newLabel
)
{
sendMessageToAllNeighbors
(
newLabel
);
...
...
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/library/PageRank.java
浏览文件 @
ce2163e6
...
...
@@ -18,8 +18,6 @@
package
org.apache.flink.graph.library
;
import
java.io.Serializable
;
import
org.apache.flink.graph.Edge
;
import
org.apache.flink.graph.Graph
;
import
org.apache.flink.graph.GraphAlgorithm
;
...
...
@@ -27,8 +25,7 @@ import org.apache.flink.graph.spargel.MessageIterator;
import
org.apache.flink.graph.spargel.MessagingFunction
;
import
org.apache.flink.graph.spargel.VertexUpdateFunction
;
public
class
PageRank
<
K
extends
Comparable
<
K
>
&
Serializable
>
implements
GraphAlgorithm
<
K
,
Double
,
Double
>
{
public
class
PageRank
<
K
>
implements
GraphAlgorithm
<
K
,
Double
,
Double
>
{
private
double
beta
;
private
int
maxIterations
;
...
...
@@ -51,8 +48,7 @@ public class PageRank<K extends Comparable<K> & Serializable> implements
* ranks from all incoming messages and then applying the dampening formula.
*/
@SuppressWarnings
(
"serial"
)
public
static
final
class
VertexRankUpdater
<
K
extends
Comparable
<
K
>
&
Serializable
>
extends
VertexUpdateFunction
<
K
,
Double
,
Double
>
{
public
static
final
class
VertexRankUpdater
<
K
>
extends
VertexUpdateFunction
<
K
,
Double
,
Double
>
{
private
final
double
beta
;
private
final
long
numVertices
;
...
...
@@ -82,8 +78,7 @@ public class PageRank<K extends Comparable<K> & Serializable> implements
* value.
*/
@SuppressWarnings
(
"serial"
)
public
static
final
class
RankMessenger
<
K
extends
Comparable
<
K
>
&
Serializable
>
extends
MessagingFunction
<
K
,
Double
,
Double
,
Double
>
{
public
static
final
class
RankMessenger
<
K
>
extends
MessagingFunction
<
K
,
Double
,
Double
,
Double
>
{
private
final
long
numVertices
;
...
...
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/library/SingleSourceShortestPaths.java
浏览文件 @
ce2163e6
...
...
@@ -27,11 +27,8 @@ import org.apache.flink.graph.spargel.MessageIterator;
import
org.apache.flink.graph.spargel.MessagingFunction
;
import
org.apache.flink.graph.spargel.VertexUpdateFunction
;
import
java.io.Serializable
;
@SuppressWarnings
(
"serial"
)
public
class
SingleSourceShortestPaths
<
K
extends
Comparable
<
K
>
&
Serializable
>
implements
GraphAlgorithm
<
K
,
Double
,
Double
>
{
public
class
SingleSourceShortestPaths
<
K
>
implements
GraphAlgorithm
<
K
,
Double
,
Double
>
{
private
final
K
srcVertexId
;
private
final
Integer
maxIterations
;
...
...
@@ -49,8 +46,7 @@ public class SingleSourceShortestPaths<K extends Comparable<K> & Serializable>
maxIterations
);
}
public
static
final
class
InitVerticesMapper
<
K
extends
Comparable
<
K
>
&
Serializable
>
implements
MapFunction
<
Vertex
<
K
,
Double
>,
Double
>
{
public
static
final
class
InitVerticesMapper
<
K
>
implements
MapFunction
<
Vertex
<
K
,
Double
>,
Double
>
{
private
K
srcVertexId
;
...
...
@@ -73,8 +69,7 @@ public class SingleSourceShortestPaths<K extends Comparable<K> & Serializable>
*
* @param <K>
*/
public
static
final
class
VertexDistanceUpdater
<
K
extends
Comparable
<
K
>
&
Serializable
>
extends
VertexUpdateFunction
<
K
,
Double
,
Double
>
{
public
static
final
class
VertexDistanceUpdater
<
K
>
extends
VertexUpdateFunction
<
K
,
Double
,
Double
>
{
@Override
public
void
updateVertex
(
K
vertexKey
,
Double
vertexValue
,
...
...
@@ -100,8 +95,7 @@ public class SingleSourceShortestPaths<K extends Comparable<K> & Serializable>
*
* @param <K>
*/
public
static
final
class
MinDistanceMessenger
<
K
extends
Comparable
<
K
>
&
Serializable
>
extends
MessagingFunction
<
K
,
Double
,
Double
,
Double
>
{
public
static
final
class
MinDistanceMessenger
<
K
>
extends
MessagingFunction
<
K
,
Double
,
Double
,
Double
>
{
@Override
public
void
sendMessages
(
K
vertexKey
,
Double
newDistance
)
...
...
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/spargel/MessagingFunction.java
浏览文件 @
ce2163e6
...
...
@@ -38,8 +38,7 @@ import org.apache.flink.util.Collector;
* @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.
*/
public
abstract
class
MessagingFunction
<
VertexKey
extends
Comparable
<
VertexKey
>
&
Serializable
,
VertexValue
extends
Serializable
,
Message
,
EdgeValue
extends
Serializable
>
implements
Serializable
{
public
abstract
class
MessagingFunction
<
VertexKey
,
VertexValue
,
Message
,
EdgeValue
>
implements
Serializable
{
private
static
final
long
serialVersionUID
=
1L
;
...
...
@@ -198,8 +197,7 @@ public abstract class MessagingFunction<VertexKey extends Comparable<VertexKey>
this
.
edgesUsed
=
false
;
}
private
static
final
class
EdgesIterator
<
VertexKey
extends
Comparable
<
VertexKey
>
&
Serializable
,
EdgeValue
extends
Serializable
>
private
static
final
class
EdgesIterator
<
VertexKey
,
EdgeValue
>
implements
Iterator
<
Edge
<
VertexKey
,
EdgeValue
>>,
Iterable
<
Edge
<
VertexKey
,
EdgeValue
>>
{
private
Iterator
<
Edge
<
VertexKey
,
EdgeValue
>>
input
;
...
...
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/spargel/VertexCentricIteration.java
浏览文件 @
ce2163e6
...
...
@@ -18,7 +18,6 @@
package
org.apache.flink.graph.spargel
;
import
java.io.Serializable
;
import
java.util.Iterator
;
import
java.util.Map
;
...
...
@@ -69,8 +68,7 @@ import org.apache.flink.util.Collector;
* @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.
*/
public
class
VertexCentricIteration
<
VertexKey
extends
Comparable
<
VertexKey
>
&
Serializable
,
VertexValue
extends
Serializable
,
Message
,
EdgeValue
extends
Serializable
>
public
class
VertexCentricIteration
<
VertexKey
,
VertexValue
,
Message
,
EdgeValue
>
implements
CustomUnaryOperation
<
Vertex
<
VertexKey
,
VertexValue
>,
Vertex
<
VertexKey
,
VertexValue
>>
{
private
final
VertexUpdateFunction
<
VertexKey
,
VertexValue
,
Message
>
updateFunction
;
...
...
@@ -218,8 +216,7 @@ public class VertexCentricIteration<VertexKey extends Comparable<VertexKey> & Se
*
* @return An in stance of the vertex-centric graph computation operator.
*/
public
static
final
<
VertexKey
extends
Comparable
<
VertexKey
>
&
Serializable
,
VertexValue
extends
Serializable
,
Message
,
EdgeValue
extends
Serializable
>
public
static
final
<
VertexKey
,
VertexValue
,
Message
,
EdgeValue
>
VertexCentricIteration
<
VertexKey
,
VertexValue
,
Message
,
EdgeValue
>
withEdges
(
DataSet
<
Edge
<
VertexKey
,
EdgeValue
>>
edgesWithValue
,
VertexUpdateFunction
<
VertexKey
,
VertexValue
,
Message
>
uf
,
...
...
@@ -233,8 +230,7 @@ public class VertexCentricIteration<VertexKey extends Comparable<VertexKey> & Se
// Wrapping UDFs
// --------------------------------------------------------------------------------------------
private
static
final
class
VertexUpdateUdf
<
VertexKey
extends
Comparable
<
VertexKey
>
&
Serializable
,
VertexValue
extends
Serializable
,
Message
>
private
static
final
class
VertexUpdateUdf
<
VertexKey
,
VertexValue
,
Message
>
extends
RichCoGroupFunction
<
Tuple2
<
VertexKey
,
Message
>,
Vertex
<
VertexKey
,
VertexValue
>,
Vertex
<
VertexKey
,
VertexValue
>>
implements
ResultTypeQueryable
<
Vertex
<
VertexKey
,
VertexValue
>>
{
...
...
@@ -308,8 +304,7 @@ public class VertexCentricIteration<VertexKey extends Comparable<VertexKey> & Se
/*
* UDF that encapsulates the message sending function for graphs where the edges have an associated value.
*/
private
static
final
class
MessagingUdfWithEdgeValues
<
VertexKey
extends
Comparable
<
VertexKey
>
&
Serializable
,
VertexValue
extends
Serializable
,
Message
,
EdgeValue
extends
Serializable
>
private
static
final
class
MessagingUdfWithEdgeValues
<
VertexKey
,
VertexValue
,
Message
,
EdgeValue
>
extends
RichCoGroupFunction
<
Edge
<
VertexKey
,
EdgeValue
>,
Vertex
<
VertexKey
,
VertexValue
>,
Tuple2
<
VertexKey
,
Message
>>
implements
ResultTypeQueryable
<
Tuple2
<
VertexKey
,
Message
>>
{
...
...
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/spargel/VertexUpdateFunction.java
浏览文件 @
ce2163e6
...
...
@@ -36,8 +36,7 @@ import org.apache.flink.util.Collector;
* <VertexValue> The vertex value type.
* <Message> The message type.
*/
public
abstract
class
VertexUpdateFunction
<
VertexKey
extends
Comparable
<
VertexKey
>
&
Serializable
,
VertexValue
extends
Serializable
,
Message
>
implements
Serializable
{
public
abstract
class
VertexUpdateFunction
<
VertexKey
,
VertexValue
,
Message
>
implements
Serializable
{
private
static
final
long
serialVersionUID
=
1L
;
...
...
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/utils/EdgeToTuple3Map.java
浏览文件 @
ce2163e6
...
...
@@ -18,14 +18,11 @@
package
org.apache.flink.graph.utils
;
import
java.io.Serializable
;
import
org.apache.flink.api.common.functions.MapFunction
;
import
org.apache.flink.api.java.tuple.Tuple3
;
import
org.apache.flink.graph.Edge
;
public
class
EdgeToTuple3Map
<
K
extends
Comparable
<
K
>
&
Serializable
,
EV
extends
Serializable
>
implements
MapFunction
<
Edge
<
K
,
EV
>,
Tuple3
<
K
,
K
,
EV
>>
{
public
class
EdgeToTuple3Map
<
K
,
EV
>
implements
MapFunction
<
Edge
<
K
,
EV
>,
Tuple3
<
K
,
K
,
EV
>>
{
private
static
final
long
serialVersionUID
=
1L
;
...
...
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/utils/Tuple2ToVertexMap.java
浏览文件 @
ce2163e6
...
...
@@ -18,14 +18,11 @@
package
org.apache.flink.graph.utils
;
import
java.io.Serializable
;
import
org.apache.flink.api.common.functions.MapFunction
;
import
org.apache.flink.api.java.tuple.Tuple2
;
import
org.apache.flink.graph.Vertex
;
public
class
Tuple2ToVertexMap
<
K
extends
Comparable
<
K
>
&
Serializable
,
VV
extends
Serializable
>
implements
MapFunction
<
Tuple2
<
K
,
VV
>,
Vertex
<
K
,
VV
>>
{
public
class
Tuple2ToVertexMap
<
K
,
VV
>
implements
MapFunction
<
Tuple2
<
K
,
VV
>,
Vertex
<
K
,
VV
>>
{
private
static
final
long
serialVersionUID
=
1L
;
...
...
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/utils/Tuple3ToEdgeMap.java
浏览文件 @
ce2163e6
...
...
@@ -18,8 +18,6 @@
package
org.apache.flink.graph.utils
;
import
java.io.Serializable
;
import
org.apache.flink.api.common.functions.MapFunction
;
import
org.apache.flink.api.java.tuple.Tuple3
;
import
org.apache.flink.graph.Edge
;
...
...
@@ -30,8 +28,7 @@ import org.apache.flink.graph.Edge;
* @param <K>
* @param <EV>
*/
public
class
Tuple3ToEdgeMap
<
K
extends
Comparable
<
K
>
&
Serializable
,
EV
extends
Serializable
>
implements
MapFunction
<
Tuple3
<
K
,
K
,
EV
>,
Edge
<
K
,
EV
>>
{
public
class
Tuple3ToEdgeMap
<
K
,
EV
>
implements
MapFunction
<
Tuple3
<
K
,
K
,
EV
>,
Edge
<
K
,
EV
>>
{
private
static
final
long
serialVersionUID
=
1L
;
...
...
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/utils/VertexToTuple2Map.java
浏览文件 @
ce2163e6
...
...
@@ -18,14 +18,11 @@
package
org.apache.flink.graph.utils
;
import
java.io.Serializable
;
import
org.apache.flink.api.common.functions.MapFunction
;
import
org.apache.flink.api.java.tuple.Tuple2
;
import
org.apache.flink.graph.Vertex
;
public
class
VertexToTuple2Map
<
K
extends
Comparable
<
K
>
&
Serializable
,
VV
extends
Serializable
>
implements
MapFunction
<
Vertex
<
K
,
VV
>,
Tuple2
<
K
,
VV
>>
{
public
class
VertexToTuple2Map
<
K
,
VV
>
implements
MapFunction
<
Vertex
<
K
,
VV
>,
Tuple2
<
K
,
VV
>>
{
private
static
final
long
serialVersionUID
=
1L
;
...
...
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/validation/GraphValidator.java
浏览文件 @
ce2163e6
...
...
@@ -30,8 +30,7 @@ import org.apache.flink.graph.Graph;
* @param <EV> the edge value type
*/
@SuppressWarnings
(
"serial"
)
public
abstract
class
GraphValidator
<
K
extends
Comparable
<
K
>
&
Serializable
,
VV
extends
Serializable
,
EV
extends
Serializable
>
implements
Serializable
{
public
abstract
class
GraphValidator
<
K
,
VV
,
EV
>
implements
Serializable
{
public
abstract
boolean
validate
(
Graph
<
K
,
VV
,
EV
>
graph
)
throws
Exception
;
...
...
flink-staging/flink-gelly/src/main/java/org/apache/flink/graph/validation/InvalidVertexIdsValidator.java
浏览文件 @
ce2163e6
...
...
@@ -28,11 +28,8 @@ import org.apache.flink.graph.Graph;
import
org.apache.flink.graph.Vertex
;
import
org.apache.flink.util.Collector
;
import
java.io.Serializable
;
@SuppressWarnings
(
"serial"
)
public
class
InvalidVertexIdsValidator
<
K
extends
Comparable
<
K
>
&
Serializable
,
VV
extends
Serializable
,
EV
extends
Serializable
>
extends
GraphValidator
<
K
,
VV
,
EV
>
{
public
class
InvalidVertexIdsValidator
<
K
,
VV
,
EV
>
extends
GraphValidator
<
K
,
VV
,
EV
>
{
/**
* Checks that the edge set input contains valid vertex Ids, i.e. that they
...
...
@@ -51,16 +48,14 @@ public class InvalidVertexIdsValidator<K extends Comparable<K> & Serializable, V
return
invalidIds
.
map
(
new
KToTupleMap
<
K
>()).
count
()
==
0
;
}
private
static
final
class
MapEdgeIds
<
K
extends
Comparable
<
K
>
&
Serializable
,
EV
extends
Serializable
>
implements
FlatMapFunction
<
Edge
<
K
,
EV
>,
Tuple1
<
K
>>
{
private
static
final
class
MapEdgeIds
<
K
,
EV
>
implements
FlatMapFunction
<
Edge
<
K
,
EV
>,
Tuple1
<
K
>>
{
public
void
flatMap
(
Edge
<
K
,
EV
>
edge
,
Collector
<
Tuple1
<
K
>>
out
)
{
out
.
collect
(
new
Tuple1
<
K
>(
edge
.
f0
));
out
.
collect
(
new
Tuple1
<
K
>(
edge
.
f1
));
}
}
private
static
final
class
GroupInvalidIds
<
K
extends
Comparable
<
K
>
&
Serializable
,
VV
extends
Serializable
>
implements
CoGroupFunction
<
Vertex
<
K
,
VV
>,
Tuple1
<
K
>,
K
>
{
private
static
final
class
GroupInvalidIds
<
K
,
VV
>
implements
CoGroupFunction
<
Vertex
<
K
,
VV
>,
Tuple1
<
K
>,
K
>
{
public
void
coGroup
(
Iterable
<
Vertex
<
K
,
VV
>>
vertexId
,
Iterable
<
Tuple1
<
K
>>
edgeId
,
Collector
<
K
>
out
)
{
if
(!(
vertexId
.
iterator
().
hasNext
()))
{
...
...
flink-staging/flink-gelly/src/test/java/org/apache/flink/graph/test/WeaklyConnectedITCase.java
已删除
100644 → 0
浏览文件 @
ddb2b347
/*
* 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.ExecutionEnvironment
;
import
org.apache.flink.graph.Graph
;
import
org.apache.flink.test.util.MultipleProgramsTestBase
;
import
org.junit.After
;
import
org.junit.Before
;
import
org.junit.Rule
;
import
org.junit.Test
;
import
org.junit.rules.TemporaryFolder
;
import
org.junit.runner.RunWith
;
import
org.junit.runners.Parameterized
;
@RunWith
(
Parameterized
.
class
)
public
class
WeaklyConnectedITCase
extends
MultipleProgramsTestBase
{
public
WeaklyConnectedITCase
(
TestExecutionMode
mode
){
super
(
mode
);
}
private
String
resultPath
;
private
String
expectedResult
;
@Rule
public
TemporaryFolder
tempFolder
=
new
TemporaryFolder
();
@Before
public
void
before
()
throws
Exception
{
resultPath
=
tempFolder
.
newFile
().
toURI
().
toString
();
}
@After
public
void
after
()
throws
Exception
{
compareResultsByLinesInMemory
(
expectedResult
,
resultPath
);
}
@Test
public
void
testWithConnectedDirected
()
throws
Exception
{
/*
* Test isWeaklyConnected() with a connected, directed graph
*/
final
ExecutionEnvironment
env
=
ExecutionEnvironment
.
getExecutionEnvironment
();
Graph
<
Long
,
Long
,
Long
>
graph
=
Graph
.
fromDataSet
(
TestGraphUtils
.
getLongLongVertexData
(
env
),
TestGraphUtils
.
getLongLongEdgeData
(
env
),
env
);
env
.
fromElements
(
graph
.
isWeaklyConnected
(
10
)).
writeAsText
(
resultPath
);
env
.
execute
();
expectedResult
=
"true\n"
;
}
@Test
public
void
testWithDisconnectedDirected
()
throws
Exception
{
/*
* Test isWeaklyConnected() with a disconnected, directed graph
*/
final
ExecutionEnvironment
env
=
ExecutionEnvironment
.
getExecutionEnvironment
();
Graph
<
Long
,
Long
,
Long
>
graph
=
Graph
.
fromDataSet
(
TestGraphUtils
.
getLongLongVertexData
(
env
),
TestGraphUtils
.
getDisconnectedLongLongEdgeData
(
env
),
env
);
env
.
fromElements
(
graph
.
isWeaklyConnected
(
10
)).
writeAsText
(
resultPath
);
env
.
execute
();
expectedResult
=
"false\n"
;
}
@Test
public
void
testWithConnectedUndirected
()
throws
Exception
{
/*
* Test isWeaklyConnected() with a connected, undirected graph
*/
final
ExecutionEnvironment
env
=
ExecutionEnvironment
.
getExecutionEnvironment
();
Graph
<
Long
,
Long
,
Long
>
graph
=
Graph
.
fromDataSet
(
TestGraphUtils
.
getLongLongVertexData
(
env
),
TestGraphUtils
.
getLongLongEdgeData
(
env
),
env
).
getUndirected
();
env
.
fromElements
(
graph
.
isWeaklyConnected
(
10
)).
writeAsText
(
resultPath
);
env
.
execute
();
expectedResult
=
"true\n"
;
}
@Test
public
void
testWithDisconnectedUndirected
()
throws
Exception
{
/*
* Test isWeaklyConnected() with a disconnected, undirected graph
*/
final
ExecutionEnvironment
env
=
ExecutionEnvironment
.
getExecutionEnvironment
();
Graph
<
Long
,
Long
,
Long
>
graph
=
Graph
.
fromDataSet
(
TestGraphUtils
.
getLongLongVertexData
(
env
),
TestGraphUtils
.
getDisconnectedLongLongEdgeData
(
env
),
env
).
getUndirected
();
env
.
fromElements
(
graph
.
isWeaklyConnected
(
10
)).
writeAsText
(
resultPath
);
env
.
execute
();
expectedResult
=
"false\n"
;
}
}
编辑
预览
Markdown
is supported
0%
请重试
或
添加新附件
.
添加附件
取消
You are about to add
0
people
to the discussion. Proceed with caution.
先完成此消息的编辑!
取消
想要评论请
注册
或
登录