Skip to content

Commit 491a879

Browse files
committed
add RuntimeContextAdapter in module fluss-flink-1.18 and fluss-flink-common
Signed-off-by: peiyu <[email protected]>
1 parent b9b92d4 commit 491a879

File tree

3 files changed

+70
-1
lines changed

3 files changed

+70
-1
lines changed
Lines changed: 34 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,34 @@
1+
/*
2+
* Licensed to the Apache Software Foundation (ASF) under one or more
3+
* contributor license agreements. See the NOTICE file distributed with
4+
* this work for additional information regarding copyright ownership.
5+
* The ASF licenses this file to You under the Apache License, Version 2.0
6+
* (the "License"); you may not use this file except in compliance with
7+
* the License. You may obtain a copy of the License at
8+
*
9+
* http://www.apache.org/licenses/LICENSE-2.0
10+
*
11+
* Unless required by applicable law or agreed to in writing, software
12+
* distributed under the License is distributed on an "AS IS" BASIS,
13+
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
14+
* See the License for the specific language governing permissions and
15+
* limitations under the License.
16+
*/
17+
18+
package org.apache.fluss.flink.adapter;
19+
20+
import org.apache.flink.api.common.functions.RuntimeContext;
21+
22+
/**
23+
* An adapter for Flink {@link RuntimeContext} class. The {@link RuntimeContext} class added the
24+
* `getJobInfo` and `getTaskInfo` methods in version 1.19 and deprecated many methods, such as
25+
* `getAttemptNumber`.
26+
*
27+
* <p>TODO: remove this class when no longer support flink 1.18.
28+
*/
29+
public class RuntimeContextAdapter {
30+
31+
public static int getAttemptNumber(RuntimeContext runtimeContext) {
32+
return runtimeContext.getAttemptNumber();
33+
}
34+
}
Lines changed: 34 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,34 @@
1+
/*
2+
* Licensed to the Apache Software Foundation (ASF) under one or more
3+
* contributor license agreements. See the NOTICE file distributed with
4+
* this work for additional information regarding copyright ownership.
5+
* The ASF licenses this file to You under the Apache License, Version 2.0
6+
* (the "License"); you may not use this file except in compliance with
7+
* the License. You may obtain a copy of the License at
8+
*
9+
* http://www.apache.org/licenses/LICENSE-2.0
10+
*
11+
* Unless required by applicable law or agreed to in writing, software
12+
* distributed under the License is distributed on an "AS IS" BASIS,
13+
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
14+
* See the License for the specific language governing permissions and
15+
* limitations under the License.
16+
*/
17+
18+
package org.apache.fluss.flink.adapter;
19+
20+
import org.apache.flink.api.common.functions.RuntimeContext;
21+
22+
/**
23+
* An adapter for Flink {@link RuntimeContext} class. The {@link RuntimeContext} class added the
24+
* `getJobInfo` and `getTaskInfo` methods in version 1.19 and deprecated many methods, such as
25+
* `getAttemptNumber`.
26+
*
27+
* <p>TODO: remove this class when no longer support flink 1.18.
28+
*/
29+
public class RuntimeContextAdapter {
30+
31+
public static int getAttemptNumber(RuntimeContext runtimeContext) {
32+
return runtimeContext.getTaskInfo().getAttemptNumber();
33+
}
34+
}

fluss-flink/fluss-flink-common/src/main/java/org/apache/fluss/flink/tiering/committer/TieringCommitOperator.java

Lines changed: 2 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -23,6 +23,7 @@
2323
import org.apache.fluss.client.metadata.LakeSnapshot;
2424
import org.apache.fluss.config.Configuration;
2525
import org.apache.fluss.exception.LakeTableSnapshotNotExistException;
26+
import org.apache.fluss.flink.adapter.RuntimeContextAdapter;
2627
import org.apache.fluss.flink.tiering.event.FailedTieringEvent;
2728
import org.apache.fluss.flink.tiering.event.FinishedTieringEvent;
2829
import org.apache.fluss.flink.tiering.event.TieringFailOverEvent;
@@ -132,7 +133,7 @@ public void setup(
132133
StreamConfig config,
133134
Output<StreamRecord<CommittableMessage<Committable>>> output) {
134135
super.setup(containingTask, config, output);
135-
int attemptNumber = getRuntimeContext().getAttemptNumber();
136+
int attemptNumber = RuntimeContextAdapter.getAttemptNumber(getRuntimeContext());
136137
if (attemptNumber > 0) {
137138
LOG.info("Send TieringFailoverEvent, current attempt number: {}", attemptNumber);
138139
// attempt number is greater than zero, the job must failover

0 commit comments

Comments
 (0)