[
https://issues.apache.org/jira/browse/FLINK-4221?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=15616493#comment-15616493
]
ASF GitHub Bot commented on FLINK-4221:
---------------------------------------
Github user zentol commented on a diff in the pull request:
https://github.com/apache/flink/pull/2724#discussion_r85606565
--- Diff:
flink-streaming-java/src/main/java/org/apache/flink/streaming/api/graph/StreamingJobGraphGenerator.java
---
@@ -383,25 +384,28 @@ private void connect(Integer headOfChain, StreamEdge
edge) {
downStreamConfig.setNumberOfInputs(downStreamConfig.getNumberOfInputs() + 1);
StreamPartitioner<?> partitioner = edge.getPartitioner();
+ JobEdge jobEdge = null;
if (partitioner instanceof ForwardPartitioner) {
- downStreamVertex.connectNewDataSetAsInput(
+ jobEdge = downStreamVertex.connectNewDataSetAsInput(
headVertex,
DistributionPattern.POINTWISE,
ResultPartitionType.PIPELINED,
true);
} else if (partitioner instanceof RescalePartitioner){
- downStreamVertex.connectNewDataSetAsInput(
+ jobEdge = downStreamVertex.connectNewDataSetAsInput(
headVertex,
DistributionPattern.POINTWISE,
ResultPartitionType.PIPELINED,
true);
} else {
- downStreamVertex.connectNewDataSetAsInput(
+ jobEdge = downStreamVertex.connectNewDataSetAsInput(
headVertex,
DistributionPattern.ALL_TO_ALL,
ResultPartitionType.PIPELINED,
true);
}
+ // set strategy name so that web interface can show it.
+ jobEdge.setShipStrategyName(partitioner.toString());
--- End diff --
are these encoded in the JSON plan returned by
`ExecutionGraph.getJsonPlan()`?
> Show metrics in WebFrontend
> ---------------------------
>
> Key: FLINK-4221
> URL: https://issues.apache.org/jira/browse/FLINK-4221
> Project: Flink
> Issue Type: Sub-task
> Components: Metrics, Webfrontend
> Affects Versions: 1.0.0
> Reporter: Chesnay Schepler
> Assignee: Robert Metzger
> Fix For: pre-apache
>
>
--
This message was sent by Atlassian JIRA
(v6.3.4#6332)