Skip to content

Commit f181e17

Browse files
committed
[FLINK-36390][core] Remove deprecated open(Configuration) method in RichFunction
1 parent be4549d commit f181e17

File tree

38 files changed

+136
-251
lines changed

38 files changed

+136
-251
lines changed

docs/content.zh/docs/dev/datastream/fault-tolerance/state.md

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -160,7 +160,7 @@ public class CountWindowAverage extends RichFlatMapFunction<Tuple2<Long, Long>,
160160
}
161161

162162
@Override
163-
public void open(Configuration config) {
163+
public void open(OpenContext ctx) {
164164
ValueStateDescriptor<Tuple2<Long, Long>> descriptor =
165165
new ValueStateDescriptor<>(
166166
"average", // the state name

docs/content.zh/docs/learn-flink/etl.md

Lines changed: 3 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -236,7 +236,7 @@ minutesByStartCell
236236

237237
对其中的每一个接口,Flink 同样提供了一个所谓 "rich" 的变体,如 `RichFlatMapFunction`,其中增加了以下方法,包括:
238238

239-
- `open(Configuration c)`
239+
- `open(OpenContext context)`
240240
- `close()`
241241
- `getRuntimeContext()`
242242

@@ -280,7 +280,7 @@ public static class Deduplicator extends RichFlatMapFunction<Event, Event> {
280280
ValueState<Boolean> keyHasBeenSeen;
281281

282282
@Override
283-
public void open(Configuration conf) {
283+
public void open(OpenContext ctx) {
284284
ValueStateDescriptor<Boolean> desc = new ValueStateDescriptor<>("keyHasBeenSeen", Types.BOOLEAN);
285285
keyHasBeenSeen = getRuntimeContext().getState(desc);
286286
}
@@ -373,7 +373,7 @@ public static class ControlFunction extends RichCoFlatMapFunction<String, String
373373
private ValueState<Boolean> blocked;
374374

375375
@Override
376-
public void open(Configuration config) {
376+
public void open(OpenContext ctx) {
377377
blocked = getRuntimeContext()
378378
.getState(new ValueStateDescriptor<>("blocked", Boolean.class));
379379
}

docs/content.zh/docs/learn-flink/event_driven.md

Lines changed: 2 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -79,7 +79,7 @@ public static class PseudoWindow extends
7979

8080
@Override
8181
// 在初始化期间调用一次。
82-
public void open(Configuration conf) {
82+
public void open(OpenContext ctx) {
8383
. . .
8484
}
8585

@@ -126,7 +126,7 @@ public static class PseudoWindow extends
126126
private transient MapState<Long, Float> sumOfTips;
127127

128128
@Override
129-
public void open(Configuration conf) {
129+
public void open(OpenContext ctx) {
130130

131131
MapStateDescriptor<Long, Float> sumDesc =
132132
new MapStateDescriptor<>("sumOfTips", Long.class, Float.class);

docs/content.zh/docs/ops/metrics.md

Lines changed: 7 additions & 7 deletions
Original file line numberDiff line numberDiff line change
@@ -52,7 +52,7 @@ public class MyMapper extends RichMapFunction<String, String> {
5252
private transient Counter counter;
5353

5454
@Override
55-
public void open(Configuration config) {
55+
public void open(OpenContext ctx) {
5656
this.counter = getRuntimeContext()
5757
.getMetricGroup()
5858
.counter("myCounter");
@@ -116,7 +116,7 @@ public class MyMapper extends RichMapFunction<String, String> {
116116
private transient Counter counter;
117117

118118
@Override
119-
public void open(Configuration config) {
119+
public void open(OpenContext ctx) {
120120
this.counter = getRuntimeContext()
121121
.getMetricGroup()
122122
.counter("myCustomCounter", new CustomCounter());
@@ -173,7 +173,7 @@ public class MyMapper extends RichMapFunction<String, String> {
173173
private transient int valueToExpose = 0;
174174

175175
@Override
176-
public void open(Configuration config) {
176+
public void open(OpenContext ctx) {
177177
getRuntimeContext()
178178
.getMetricGroup()
179179
.gauge("MyGauge", new Gauge<Integer>() {
@@ -247,7 +247,7 @@ public class MyMapper extends RichMapFunction<Long, Long> {
247247
private transient Histogram histogram;
248248

249249
@Override
250-
public void open(Configuration config) {
250+
public void open(OpenContext ctx) {
251251
this.histogram = getRuntimeContext()
252252
.getMetricGroup()
253253
.histogram("myHistogram", new MyHistogram());
@@ -307,7 +307,7 @@ public class MyMapper extends RichMapFunction<Long, Long> {
307307
private transient Histogram histogram;
308308

309309
@Override
310-
public void open(Configuration config) {
310+
public void open(OpenContext ctx) {
311311
com.codahale.metrics.Histogram dropwizardHistogram =
312312
new com.codahale.metrics.Histogram(new SlidingWindowReservoir(500));
313313

@@ -366,7 +366,7 @@ public class MyMapper extends RichMapFunction<Long, Long> {
366366
private transient Meter meter;
367367

368368
@Override
369-
public void open(Configuration config) {
369+
public void open(OpenContext ctx) {
370370
this.meter = getRuntimeContext()
371371
.getMetricGroup()
372372
.meter("myMeter", new MyMeter());
@@ -440,7 +440,7 @@ public class MyMapper extends RichMapFunction<Long, Long> {
440440
private transient Meter meter;
441441

442442
@Override
443-
public void open(Configuration config) {
443+
public void open(OpenContext ctx) {
444444
com.codahale.metrics.Meter dropwizardMeter = new com.codahale.metrics.Meter();
445445

446446
this.meter = getRuntimeContext()

docs/content/docs/dev/datastream/fault-tolerance/state.md

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -192,7 +192,7 @@ public class CountWindowAverage extends RichFlatMapFunction<Tuple2<Long, Long>,
192192
}
193193

194194
@Override
195-
public void open(Configuration config) {
195+
public void open(OpenContext ctx) {
196196
ValueStateDescriptor<Tuple2<Long, Long>> descriptor =
197197
new ValueStateDescriptor<>(
198198
"average", // the state name

docs/content/docs/learn-flink/etl.md

Lines changed: 3 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -274,7 +274,7 @@ Abstract Method pattern.
274274
For each of these interfaces, Flink also provides a so-called "rich" variant, e.g.,
275275
`RichFlatMapFunction`, which has some additional methods, including:
276276

277-
- `open(Configuration c)`
277+
- `open(OpenContext context)`
278278
- `close()`
279279
- `getRuntimeContext()`
280280

@@ -329,7 +329,7 @@ public static class Deduplicator extends RichFlatMapFunction<Event, Event> {
329329
ValueState<Boolean> keyHasBeenSeen;
330330

331331
@Override
332-
public void open(Configuration conf) {
332+
public void open(OpenContext ctx) {
333333
ValueStateDescriptor<Boolean> desc = new ValueStateDescriptor<>("keyHasBeenSeen", Types.BOOLEAN);
334334
keyHasBeenSeen = getRuntimeContext().getState(desc);
335335
}
@@ -447,7 +447,7 @@ public static class ControlFunction extends RichCoFlatMapFunction<String, String
447447
private ValueState<Boolean> blocked;
448448

449449
@Override
450-
public void open(Configuration config) {
450+
public void open(OpenContext ctx) {
451451
blocked = getRuntimeContext()
452452
.getState(new ValueStateDescriptor<>("blocked", Boolean.class));
453453
}

docs/content/docs/learn-flink/event_driven.md

Lines changed: 2 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -75,7 +75,7 @@ public static class PseudoWindow extends
7575

7676
@Override
7777
// Called once during initialization.
78-
public void open(Configuration conf) {
78+
public void open(OpenContext ctx) {
7979
. . .
8080
}
8181

@@ -116,7 +116,7 @@ Things to be aware of:
116116
private transient MapState<Long, Float> sumOfTips;
117117

118118
@Override
119-
public void open(Configuration conf) {
119+
public void open(OpenContext ctx) {
120120

121121
MapStateDescriptor<Long, Float> sumDesc =
122122
new MapStateDescriptor<>("sumOfTips", Long.class, Float.class);

docs/content/docs/ops/metrics.md

Lines changed: 7 additions & 7 deletions
Original file line numberDiff line numberDiff line change
@@ -52,7 +52,7 @@ public class MyMapper extends RichMapFunction<String, String> {
5252
private transient Counter counter;
5353

5454
@Override
55-
public void open(Configuration config) {
55+
public void open(OpenContext ctx) {
5656
this.counter = getRuntimeContext()
5757
.getMetricGroup()
5858
.counter("myCounter");
@@ -116,7 +116,7 @@ public class MyMapper extends RichMapFunction<String, String> {
116116
private transient Counter counter;
117117

118118
@Override
119-
public void open(Configuration config) {
119+
public void open(OpenContext ctx) {
120120
this.counter = getRuntimeContext()
121121
.getMetricGroup()
122122
.counter("myCustomCounter", new CustomCounter());
@@ -173,7 +173,7 @@ public class MyMapper extends RichMapFunction<String, String> {
173173
private transient int valueToExpose = 0;
174174

175175
@Override
176-
public void open(Configuration config) {
176+
public void open(OpenContext ctx) {
177177
getRuntimeContext()
178178
.getMetricGroup()
179179
.gauge("MyGauge", new Gauge<Integer>() {
@@ -247,7 +247,7 @@ public class MyMapper extends RichMapFunction<Long, Long> {
247247
private transient Histogram histogram;
248248

249249
@Override
250-
public void open(Configuration config) {
250+
public void open(OpenContext ctx) {
251251
this.histogram = getRuntimeContext()
252252
.getMetricGroup()
253253
.histogram("myHistogram", new MyHistogram());
@@ -307,7 +307,7 @@ public class MyMapper extends RichMapFunction<Long, Long> {
307307
private transient Histogram histogram;
308308

309309
@Override
310-
public void open(Configuration config) {
310+
public void open(OpenContext ctx) {
311311
com.codahale.metrics.Histogram dropwizardHistogram =
312312
new com.codahale.metrics.Histogram(new SlidingWindowReservoir(500));
313313

@@ -366,7 +366,7 @@ public class MyMapper extends RichMapFunction<Long, Long> {
366366
private transient Meter meter;
367367

368368
@Override
369-
public void open(Configuration config) {
369+
public void open(OpenContext ctx) {
370370
this.meter = getRuntimeContext()
371371
.getMetricGroup()
372372
.meter("myMeter", new MyMeter());
@@ -440,7 +440,7 @@ public class MyMapper extends RichMapFunction<Long, Long> {
440440
private transient Meter meter;
441441

442442
@Override
443-
public void open(Configuration config) {
443+
public void open(OpenContext ctx) {
444444
com.codahale.metrics.Meter dropwizardMeter = new com.codahale.metrics.Meter();
445445

446446
this.meter = getRuntimeContext()

flink-core/src/main/java/org/apache/flink/api/common/functions/AbstractRichFunction.java

Lines changed: 1 addition & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -19,7 +19,6 @@
1919
package org.apache.flink.api.common.functions;
2020

2121
import org.apache.flink.annotation.Public;
22-
import org.apache.flink.configuration.Configuration;
2322

2423
import java.io.Serializable;
2524

@@ -69,7 +68,7 @@ public IterationRuntimeContext getIterationRuntimeContext() {
6968
// --------------------------------------------------------------------------------------------
7069

7170
@Override
72-
public void open(Configuration parameters) throws Exception {}
71+
public void open(OpenContext openContext) throws Exception {}
7372

7473
@Override
7574
public void close() throws Exception {}

flink-core/src/main/java/org/apache/flink/api/common/functions/BroadcastVariableInitializer.java

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -41,7 +41,7 @@
4141
*
4242
* private Map<Long, String> map;
4343
*
44-
* public void open(Configuration cfg) throws Exception {
44+
* public void open(OpenContext ctx) throws Exception {
4545
* getRuntimeContext().getBroadcastVariableWithInitializer("mapvar",
4646
* new BroadcastVariableInitializer<Tuple2<Long, String>, Map<Long, String>>() {
4747
*

0 commit comments

Comments
 (0)