-
Notifications
You must be signed in to change notification settings - Fork 13k
/
EdgeManagerBuildUtil.java
143 lines (117 loc) · 6.02 KB
/
EdgeManagerBuildUtil.java
1
2
3
4
5
6
7
8
9
10
11
12
13
14
15
16
17
18
19
20
21
22
23
24
25
26
27
28
29
30
31
32
33
34
35
36
37
38
39
40
41
42
43
44
45
46
47
48
49
50
51
52
53
54
55
56
57
58
59
60
61
62
63
64
65
66
67
68
69
70
71
72
73
74
75
76
77
78
79
80
81
82
83
84
85
86
87
88
89
90
91
92
93
94
95
96
97
98
99
100
101
102
103
104
105
106
107
108
109
110
111
112
113
114
115
116
117
118
119
120
121
122
123
124
125
126
127
128
129
130
131
132
133
134
135
136
137
138
139
140
141
142
143
/*
* 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.runtime.executiongraph;
import org.apache.flink.runtime.jobgraph.DistributionPattern;
import org.apache.flink.runtime.jobgraph.IntermediateResultPartitionID;
import org.apache.flink.runtime.scheduler.strategy.ConsumedPartitionGroup;
import org.apache.flink.runtime.scheduler.strategy.ConsumerVertexGroup;
import org.apache.flink.runtime.scheduler.strategy.ExecutionVertexID;
import java.util.ArrayList;
import java.util.Arrays;
import java.util.List;
import java.util.stream.Collectors;
/** Utilities for building {@link EdgeManager}. */
public class EdgeManagerBuildUtil {
public static void connectVertexToResult(
ExecutionJobVertex vertex,
IntermediateResult ires,
int inputNumber,
DistributionPattern distributionPattern) {
switch (distributionPattern) {
case POINTWISE:
connectPointwise(vertex.getTaskVertices(), ires, inputNumber);
break;
case ALL_TO_ALL:
connectAllToAll(vertex.getTaskVertices(), ires, inputNumber);
break;
default:
throw new RuntimeException("Unrecognized distribution pattern.");
}
}
private static void connectAllToAll(
ExecutionVertex[] taskVertices, IntermediateResult ires, int inputNumber) {
ConsumedPartitionGroup consumedPartitions =
new ConsumedPartitionGroup(
Arrays.stream(ires.getPartitions())
.map(IntermediateResultPartition::getPartitionId)
.collect(Collectors.toList()));
for (ExecutionVertex ev : taskVertices) {
ev.addConsumedPartitions(consumedPartitions, inputNumber);
}
ConsumerVertexGroup vertices =
new ConsumerVertexGroup(
Arrays.stream(taskVertices)
.map(ExecutionVertex::getID)
.collect(Collectors.toList()));
for (IntermediateResultPartition partition : ires.getPartitions()) {
partition.addConsumers(vertices);
}
}
private static void connectPointwise(
ExecutionVertex[] taskVertices, IntermediateResult ires, int inputNumber) {
final int sourceCount = ires.getPartitions().length;
final int targetCount = taskVertices.length;
if (sourceCount == targetCount) {
for (int i = 0; i < sourceCount; i++) {
ExecutionVertex executionVertex = taskVertices[i];
IntermediateResultPartition partition = ires.getPartitions()[i];
ConsumerVertexGroup consumerVertexGroup =
new ConsumerVertexGroup(executionVertex.getID());
partition.addConsumers(consumerVertexGroup);
ConsumedPartitionGroup consumedPartitionGroup =
new ConsumedPartitionGroup(partition.getPartitionId());
executionVertex.addConsumedPartitions(consumedPartitionGroup, inputNumber);
}
} else if (sourceCount > targetCount) {
for (int index = 0; index < targetCount; index++) {
ExecutionVertex executionVertex = taskVertices[index];
ConsumerVertexGroup consumerVertexGroup =
new ConsumerVertexGroup(executionVertex.getID());
int start = index * sourceCount / targetCount;
int end = (index + 1) * sourceCount / targetCount;
List<IntermediateResultPartitionID> consumedPartitions =
new ArrayList<>(end - start);
for (int i = start; i < end; i++) {
IntermediateResultPartition partition = ires.getPartitions()[i];
partition.addConsumers(consumerVertexGroup);
consumedPartitions.add(partition.getPartitionId());
}
ConsumedPartitionGroup consumedPartitionGroup =
new ConsumedPartitionGroup(consumedPartitions);
executionVertex.addConsumedPartitions(consumedPartitionGroup, inputNumber);
}
} else {
for (int partitionNum = 0; partitionNum < sourceCount; partitionNum++) {
IntermediateResultPartition partition = ires.getPartitions()[partitionNum];
ConsumedPartitionGroup consumerPartitionGroup =
new ConsumedPartitionGroup(partition.getPartitionId());
float factor = ((float) targetCount) / sourceCount;
int start = (int) (Math.ceil(partitionNum * factor));
int end = (int) (Math.ceil((partitionNum + 1) * factor));
List<ExecutionVertexID> consumers = new ArrayList<>(end - start);
for (int i = start; i < end; i++) {
ExecutionVertex executionVertex = taskVertices[i];
executionVertex.addConsumedPartitions(consumerPartitionGroup, inputNumber);
consumers.add(executionVertex.getID());
}
ConsumerVertexGroup consumerVertexGroup = new ConsumerVertexGroup(consumers);
partition.addConsumers(consumerVertexGroup);
}
}
}
}