[jira] [Updated] (FLINK-29492) Kafka exactly-once sink causes OutOfMemoryError

2022-09-30 Thread Robert Metzger (Jira)


 [ 
https://issues.apache.org/jira/browse/FLINK-29492?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Robert Metzger updated FLINK-29492:
---
Description: 
My Kafka exactly-once sinks are periodically failing with a {{OutOfMemoryError: 
Java heap space}}.

This looks very similar to FLINK-28250. But I am running 1.15.2, which contains 
a fix for FLINK-28250.

Exception:
{code:java}
java.io.IOException: Could not perform checkpoint 2281 for operator 
http_events[3]: Writer (1/1)#1.
at 
org.apache.flink.streaming.runtime.tasks.StreamTask.triggerCheckpointOnBarrier(StreamTask.java:1210)
at 
org.apache.flink.streaming.runtime.io.checkpointing.CheckpointBarrierHandler.notifyCheckpoint(CheckpointBarrierHandler.java:147)
at 
org.apache.flink.streaming.runtime.io.checkpointing.SingleCheckpointBarrierHandler.triggerCheckpoint(SingleCheckpointBarrierHandler.java:287)
at 
org.apache.flink.streaming.runtime.io.checkpointing.SingleCheckpointBarrierHandler.access$100(SingleCheckpointBarrierHandler.java:64)
at 
org.apache.flink.streaming.runtime.io.checkpointing.SingleCheckpointBarrierHandler$ControllerImpl.triggerGlobalCheckpoint(SingleCheckpointBarrierHandler.java:493)
at 
org.apache.flink.streaming.runtime.io.checkpointing.AbstractAlignedBarrierHandlerState.triggerGlobalCheckpoint(AbstractAlignedBarrierHandlerState.java:74)
at 
org.apache.flink.streaming.runtime.io.checkpointing.AbstractAlignedBarrierHandlerState.barrierReceived(AbstractAlignedBarrierHandlerState.java:66)
at 
org.apache.flink.streaming.runtime.io.checkpointing.SingleCheckpointBarrierHandler.lambda$processBarrier$2(SingleCheckpointBarrierHandler.java:234)
at 
org.apache.flink.streaming.runtime.io.checkpointing.SingleCheckpointBarrierHandler.markCheckpointAlignedAndTransformState(SingleCheckpointBarrierHandler.java:262)
at 
org.apache.flink.streaming.runtime.io.checkpointing.SingleCheckpointBarrierHandler.processBarrier(SingleCheckpointBarrierHandler.java:231)
at 
org.apache.flink.streaming.runtime.io.checkpointing.CheckpointedInputGate.handleEvent(CheckpointedInputGate.java:181)
at 
org.apache.flink.streaming.runtime.io.checkpointing.CheckpointedInputGate.pollNext(CheckpointedInputGate.java:159)
at 
org.apache.flink.streaming.runtime.io.AbstractStreamTaskNetworkInput.emitNext(AbstractStreamTaskNetworkInput.java:110)
at 
org.apache.flink.streaming.runtime.io.StreamOneInputProcessor.processInput(StreamOneInputProcessor.java:65)
at 
org.apache.flink.streaming.runtime.tasks.StreamTask.processInput(StreamTask.java:519)
at 
org.apache.flink.streaming.runtime.tasks.mailbox.MailboxProcessor.runMailboxLoop(MailboxProcessor.java:203)
at 
org.apache.flink.streaming.runtime.tasks.StreamTask.runMailboxLoop(StreamTask.java:804)
at 
org.apache.flink.streaming.runtime.tasks.StreamTask.invoke(StreamTask.java:753)
at 
org.apache.flink.runtime.taskmanager.Task.runWithSystemExitMonitoring(Task.java:948)
at 
org.apache.flink.runtime.taskmanager.Task.restoreAndInvoke(Task.java:927)
at org.apache.flink.runtime.taskmanager.Task.doRun(Task.java:741)
at org.apache.flink.runtime.taskmanager.Task.run(Task.java:563)
at java.base/java.lang.Thread.run(Unknown Source)
Caused by: org.apache.flink.runtime.checkpoint.CheckpointException: Could not 
complete snapshot 2281 for operator http_events[3]: Writer (1/1)#1. Failure 
reason: Checkpoint was declined.
at 
org.apache.flink.streaming.api.operators.StreamOperatorStateHandler.snapshotState(StreamOperatorStateHandler.java:269)
at 
org.apache.flink.streaming.api.operators.StreamOperatorStateHandler.snapshotState(StreamOperatorStateHandler.java:173)
at 
org.apache.flink.streaming.api.operators.AbstractStreamOperator.snapshotState(AbstractStreamOperator.java:348)
at 
org.apache.flink.streaming.runtime.tasks.RegularOperatorChain.checkpointStreamOperator(RegularOperatorChain.java:227)
at 
org.apache.flink.streaming.runtime.tasks.RegularOperatorChain.buildOperatorSnapshotFutures(RegularOperatorChain.java:212)
at 
org.apache.flink.streaming.runtime.tasks.RegularOperatorChain.snapshotState(RegularOperatorChain.java:192)
at 
org.apache.flink.streaming.runtime.tasks.SubtaskCheckpointCoordinatorImpl.takeSnapshotSync(SubtaskCheckpointCoordinatorImpl.java:647)
at 
org.apache.flink.streaming.runtime.tasks.SubtaskCheckpointCoordinatorImpl.checkpointState(SubtaskCheckpointCoordinatorImpl.java:320)
at 
org.apache.flink.streaming.runtime.tasks.StreamTask.lambda$performCheckpoint$12(StreamTask.java:1253)
at 
org.apache.flink.streaming.runtime.tasks.StreamTaskActionExecutor$1.runThrowing(StreamTaskActionExecutor.java:50)
at 
org.apache.flink.streaming.runtime.tasks.StreamTask.performCheckpoint(StreamTask.java:1241)
at 

[jira] [Commented] (FLINK-28250) exactly-once sink kafka cause out of memory

2022-09-30 Thread Robert Metzger (Jira)


[ 
https://issues.apache.org/jira/browse/FLINK-28250?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17611829#comment-17611829
 ] 

Robert Metzger commented on FLINK-28250:


Thanks a lot for the fix.
I've opened a related ticket here: 
https://issues.apache.org/jira/browse/FLINK-29492

> exactly-once sink kafka cause out of memory
> ---
>
> Key: FLINK-28250
> URL: https://issues.apache.org/jira/browse/FLINK-28250
> Project: Flink
>  Issue Type: Bug
>  Components: Connectors / Kafka
>Affects Versions: 1.15.0
> Environment: *flink version: flink-1.15.0*
> *tm: 8* parallelism, 1 slot, 2g
> centos7
>Reporter: jinshuangxian
>Assignee: Chalres Tan
>Priority: Critical
>  Labels: pull-request-available
> Fix For: 1.16.0, 1.15.2
>
> Attachments: image-2022-06-25-22-07-35-686.png, 
> image-2022-06-25-22-07-54-649.png, image-2022-06-25-22-08-04-891.png, 
> image-2022-06-25-22-08-15-024.png
>
>
> *my sql code:*
> CREATE TABLE sourceTable (
> data bytes
> )WITH(
> 'connector'='kafka',
> 'topic'='topic1',
> 'properties.bootstrap.servers'='host1',
> 'properties.group.id'='gorup1',
> 'scan.startup.mode'='latest-offset',
> 'format'='raw'
> );
>  
> CREATE TABLE sinkTable (
> data bytes
> )
> WITH (
> 'connector'='kafka',
> 'topic'='topic2',
> 'properties.bootstrap.servers'='host2',
> 'properties.transaction.timeout.ms'='3',
> 'sink.semantic'='exactly-once',
> 'sink.transactional-id-prefix'='xx-kafka-sink-a',
> 'format'='raw'
> );
> insert into sinkTable
> select data
> from sourceTable;
>  
> *problem:*
> After the program runs online for about half an hour, full gc frequently 
> appears
>  
> {*}Troubleshoot{*}:
> I use command 'jmap -dump:live,format=b,file=/tmp/dump2.hprof' dump the 
> problem tm memory. It is found that there are 115 
> FlinkKafkaInternalProducers, which is not normal.
> !image-2022-06-25-22-07-54-649.png!!image-2022-06-25-22-07-35-686.png!
> After reading the code of KafkaCommitter, it is found that after the commit 
> is successful, the producer is not recycled, only abnormal situations are 
> recycled.
> !image-2022-06-25-22-08-04-891.png!
> I added a few lines of code. After the online test, the program works 
> normally, and the problem of oom memory is solved.
> !image-2022-06-25-22-08-15-024.png!



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Created] (FLINK-29492) Kafka exactly-once sink causes OutOfMemoryError

2022-09-30 Thread Robert Metzger (Jira)
Robert Metzger created FLINK-29492:
--

 Summary: Kafka exactly-once sink causes OutOfMemoryError
 Key: FLINK-29492
 URL: https://issues.apache.org/jira/browse/FLINK-29492
 Project: Flink
  Issue Type: Bug
  Components: Connectors / Kafka
Affects Versions: 1.15.2
Reporter: Robert Metzger


My Kafka exactly-once sinks are periodically failing with a `OutOfMemoryError: 
Java heap space`.

This looks very similar to FLINK-28250. But I am running 1.15.2, which contains 
a fix for FLINK-28250.

Exception:
{code:java}
java.io.IOException: Could not perform checkpoint 2281 for operator 
http_events[3]: Writer (1/1)#1.
at 
org.apache.flink.streaming.runtime.tasks.StreamTask.triggerCheckpointOnBarrier(StreamTask.java:1210)
at 
org.apache.flink.streaming.runtime.io.checkpointing.CheckpointBarrierHandler.notifyCheckpoint(CheckpointBarrierHandler.java:147)
at 
org.apache.flink.streaming.runtime.io.checkpointing.SingleCheckpointBarrierHandler.triggerCheckpoint(SingleCheckpointBarrierHandler.java:287)
at 
org.apache.flink.streaming.runtime.io.checkpointing.SingleCheckpointBarrierHandler.access$100(SingleCheckpointBarrierHandler.java:64)
at 
org.apache.flink.streaming.runtime.io.checkpointing.SingleCheckpointBarrierHandler$ControllerImpl.triggerGlobalCheckpoint(SingleCheckpointBarrierHandler.java:493)
at 
org.apache.flink.streaming.runtime.io.checkpointing.AbstractAlignedBarrierHandlerState.triggerGlobalCheckpoint(AbstractAlignedBarrierHandlerState.java:74)
at 
org.apache.flink.streaming.runtime.io.checkpointing.AbstractAlignedBarrierHandlerState.barrierReceived(AbstractAlignedBarrierHandlerState.java:66)
at 
org.apache.flink.streaming.runtime.io.checkpointing.SingleCheckpointBarrierHandler.lambda$processBarrier$2(SingleCheckpointBarrierHandler.java:234)
at 
org.apache.flink.streaming.runtime.io.checkpointing.SingleCheckpointBarrierHandler.markCheckpointAlignedAndTransformState(SingleCheckpointBarrierHandler.java:262)
at 
org.apache.flink.streaming.runtime.io.checkpointing.SingleCheckpointBarrierHandler.processBarrier(SingleCheckpointBarrierHandler.java:231)
at 
org.apache.flink.streaming.runtime.io.checkpointing.CheckpointedInputGate.handleEvent(CheckpointedInputGate.java:181)
at 
org.apache.flink.streaming.runtime.io.checkpointing.CheckpointedInputGate.pollNext(CheckpointedInputGate.java:159)
at 
org.apache.flink.streaming.runtime.io.AbstractStreamTaskNetworkInput.emitNext(AbstractStreamTaskNetworkInput.java:110)
at 
org.apache.flink.streaming.runtime.io.StreamOneInputProcessor.processInput(StreamOneInputProcessor.java:65)
at 
org.apache.flink.streaming.runtime.tasks.StreamTask.processInput(StreamTask.java:519)
at 
org.apache.flink.streaming.runtime.tasks.mailbox.MailboxProcessor.runMailboxLoop(MailboxProcessor.java:203)
at 
org.apache.flink.streaming.runtime.tasks.StreamTask.runMailboxLoop(StreamTask.java:804)
at 
org.apache.flink.streaming.runtime.tasks.StreamTask.invoke(StreamTask.java:753)
at 
org.apache.flink.runtime.taskmanager.Task.runWithSystemExitMonitoring(Task.java:948)
at 
org.apache.flink.runtime.taskmanager.Task.restoreAndInvoke(Task.java:927)
at org.apache.flink.runtime.taskmanager.Task.doRun(Task.java:741)
at org.apache.flink.runtime.taskmanager.Task.run(Task.java:563)
at java.base/java.lang.Thread.run(Unknown Source)
Caused by: org.apache.flink.runtime.checkpoint.CheckpointException: Could not 
complete snapshot 2281 for operator http_events[3]: Writer (1/1)#1. Failure 
reason: Checkpoint was declined.
at 
org.apache.flink.streaming.api.operators.StreamOperatorStateHandler.snapshotState(StreamOperatorStateHandler.java:269)
at 
org.apache.flink.streaming.api.operators.StreamOperatorStateHandler.snapshotState(StreamOperatorStateHandler.java:173)
at 
org.apache.flink.streaming.api.operators.AbstractStreamOperator.snapshotState(AbstractStreamOperator.java:348)
at 
org.apache.flink.streaming.runtime.tasks.RegularOperatorChain.checkpointStreamOperator(RegularOperatorChain.java:227)
at 
org.apache.flink.streaming.runtime.tasks.RegularOperatorChain.buildOperatorSnapshotFutures(RegularOperatorChain.java:212)
at 
org.apache.flink.streaming.runtime.tasks.RegularOperatorChain.snapshotState(RegularOperatorChain.java:192)
at 
org.apache.flink.streaming.runtime.tasks.SubtaskCheckpointCoordinatorImpl.takeSnapshotSync(SubtaskCheckpointCoordinatorImpl.java:647)
at 
org.apache.flink.streaming.runtime.tasks.SubtaskCheckpointCoordinatorImpl.checkpointState(SubtaskCheckpointCoordinatorImpl.java:320)
at 
org.apache.flink.streaming.runtime.tasks.StreamTask.lambda$performCheckpoint$12(StreamTask.java:1253)
at 
org.apache.flink.streaming.runtime.t

[jira] [Created] (FLINK-29491) Primary key without partition field can be supported from full changelog

2022-09-30 Thread Jingsong Lee (Jira)
Jingsong Lee created FLINK-29491:


 Summary: Primary key without partition field can be supported from 
full changelog
 Key: FLINK-29491
 URL: https://issues.apache.org/jira/browse/FLINK-29491
 Project: Flink
  Issue Type: Improvement
  Components: Table Store
Reporter: Jingsong Lee
 Fix For: table-store-0.3.0


When pk does not contain partition fields, an exception will be thrown under 
any circumstances. We can relax this restriction. When the input is a complete 
changelog.



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Created] (FLINK-29490) Timestamp LTZ is unsupported in table store

2022-09-30 Thread Jingsong Lee (Jira)
Jingsong Lee created FLINK-29490:


 Summary: Timestamp LTZ is unsupported in table store 
 Key: FLINK-29490
 URL: https://issues.apache.org/jira/browse/FLINK-29490
 Project: Flink
  Issue Type: Bug
  Components: Table Store
Reporter: Jingsong Lee
 Fix For: table-store-0.3.0


Due to orc format limitation, timestamp ltz is unsupported now. We should fix 
this, and validate this type cross multiple engines (hive spark trino).
We need to careful about time zone.



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[GitHub] [flink] flinkbot commented on pull request #20935: [hotfix][doc] Make referencing custom image clearer for Docker

2022-09-30 Thread GitBox


flinkbot commented on PR #20935:
URL: https://github.com/apache/flink/pull/20935#issuecomment-1264115737

   
   ## CI report:
   
   * 03bf998619ad1f5ceae2eb1c067a27837c7b0b37 UNKNOWN
   
   
   Bot commands
 The @flinkbot bot supports the following commands:
   
- `@flinkbot run azure` re-run the last Azure build
   


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



[GitHub] [flink] liuml07 commented on pull request #20935: [hotfix][doc] Make referencing custom image clearer for Docker

2022-09-30 Thread GitBox


liuml07 commented on PR #20935:
URL: https://github.com/apache/flink/pull/20935#issuecomment-1264113384

   CC @knaufk @MartijnVisser


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



[GitHub] [flink] liuml07 opened a new pull request, #20935: [hotfix][doc] Make referencing custom image clearer for Docker

2022-09-30 Thread GitBox


liuml07 opened a new pull request, #20935:
URL: https://github.com/apache/flink/pull/20935

   ## What is the purpose of the change
   
   Make referencing custom image clearer for Docker
   
   ## Brief change log
   
   hotfix
   
   ## Verifying this change
   
   Please make sure both new and modified tests in this PR follows the 
conventions defined in our code quality guide: 
https://flink.apache.org/contributing/code-style-and-quality-common.html#testing
   
   This change is a trivial rework / code cleanup without any test coverage.
   
   ## Does this pull request potentially affect one of the following parts:
   
 - Dependencies (does it add or upgrade a dependency): (yes / **no**)
 - The public API, i.e., is any changed class annotated with 
`@Public(Evolving)`: (yes / **no**)
 - The serializers: (yes / **no** / don't know)
 - The runtime per-record code paths (performance sensitive): (yes / **no** 
/ don't know)
 - Anything that affects deployment or recovery: JobManager (and its 
components), Checkpointing, Kubernetes/Yarn, ZooKeeper: (yes / **no** / don't 
know)
 - The S3 file system connector: (yes / **no** / don't know)
   
   ## Documentation
   
 - Does this pull request introduce a new feature? (yes / **no**)
 - If yes, how is the feature documented? (**not applicable** / docs / 
JavaDocs / not documented)
   


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



[GitHub] [flink] stevenzwu commented on pull request #20852: [FLINK-27101][checkpointing][rest] Add restful API to trigger checkpoints

2022-09-30 Thread GitBox


stevenzwu commented on PR #20852:
URL: https://github.com/apache/flink/pull/20852#issuecomment-1264009972

   @leletan we shall come back to this PR after the internal class renaming is 
merged. Please create a jira and PR for that first.


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



[GitHub] [flink] leletan commented on pull request #20852: [FLINK-27101][checkpointing][rest] Add restful API to trigger checkpoints

2022-09-30 Thread GitBox


leletan commented on PR #20852:
URL: https://github.com/apache/flink/pull/20852#issuecomment-1263952346

   As per discussion 
https://github.com/apache/flink/pull/20852#issuecomment-1260402519, will 
address the internal class renaming in a follow up PR.


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



[GitHub] [flink] leletan commented on pull request #20852: [FLINK-27101][checkpointing][rest] Add restful API to trigger checkpoints

2022-09-30 Thread GitBox


leletan commented on PR #20852:
URL: https://github.com/apache/flink/pull/20852#issuecomment-1263951145

   @pnowojski Your comments and change requests should all be addressed. Please 
let me know if I miss anything. Thanks!


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



[GitHub] [flink] leletan commented on pull request #20852: [FLINK-27101][checkpointing][rest] Add restful API to trigger checkpoints

2022-09-30 Thread GitBox


leletan commented on PR #20852:
URL: https://github.com/apache/flink/pull/20852#issuecomment-1263949515

   @stevenzwu All your comments should be addressed. Please let me know if 
there is anything missing. Thanks!


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



[GitHub] [flink] leletan commented on a diff in pull request #20852: [FLINK-27101][checkpointing][rest] Add restful API to trigger checkpoints

2022-09-30 Thread GitBox


leletan commented on code in PR #20852:
URL: https://github.com/apache/flink/pull/20852#discussion_r984896568


##
flink-core/src/main/java/org/apache/flink/core/execution/CheckpointType.java:
##
@@ -0,0 +1,52 @@
+/*
+ * 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.core.execution;
+
+import org.apache.flink.annotation.Internal;
+import org.apache.flink.annotation.PublicEvolving;
+import org.apache.flink.configuration.DescribedEnum;
+import org.apache.flink.configuration.description.InlineElement;
+
+import static org.apache.flink.configuration.description.TextElement.text;
+
+/** Describes the type in which a checkpoint should be taken. */
+@PublicEvolving
+public enum CheckpointType implements DescribedEnum {
+/** A checkpoint type with the entire state check pointed, common for all 
state backends. */
+FULL("A checkpoint type with the entire state check pointed, common for 
all state backends."),
+
+/**
+ * A checkpoint type that check points only the difference between 
snapshots, specific for
+ * certain state backend.
+ */
+INCREMENTAL(
+"A checkpoint type that check points only the difference between 
snapshots, specific for certain state backend.");
+private final InlineElement description;
+public static final CheckpointType DEFAULT = CheckpointType.FULL;
+
+CheckpointType(String description) {
+this.description = text(description);
+}
+
+@Override
+@Internal

Review Comment:
   This is copied from 
[SavepointFormatType](https://github.com/apache/flink/blob/f543b8ac690b1dee58bc3cb345a1c8ad0db0941e/flink-core/src/main/java/org/apache/flink/core/execution/SavepointFormatType.java#L50)
   But I am fine with removing it.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



[GitHub] [flink] leletan commented on a diff in pull request #20852: [FLINK-27101][checkpointing][rest] Add restful API to trigger checkpoints

2022-09-30 Thread GitBox


leletan commented on code in PR #20852:
URL: https://github.com/apache/flink/pull/20852#discussion_r984896981


##
flink-runtime/src/main/java/org/apache/flink/runtime/checkpoint/CheckpointCoordinator.java:
##
@@ -511,6 +514,42 @@ public CompletableFuture 
triggerCheckpoint(boolean isPeriod
 return triggerCheckpointFromCheckpointThread(checkpointProperties, 
null, isPeriodic);
 }
 
+/**
+ * Triggers one new checkpoint with the given checkpointType. If the given 
checkpointType is
+ * null, then it will fall back to use the CheckpointCoordinator's 
checkpointProperties. The
+ * return value is a future. It completes when the checkpoint triggered 
finishes or an error
+ * occurred.
+ *
+ * @param checkpointType specifies the back up type of the checkpoint to 
trigger.
+ * @return a future to the completed checkpoint.
+ */
+public CompletableFuture triggerCheckpoint(
+@Nullable CheckpointType checkpointType) {
+
+final SnapshotType snapshotType;
+if (checkpointType == null) {
+snapshotType = checkpointProperties.getCheckpointType();

Review Comment:
   this will be handled in a follow up PR as discussed.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



[GitHub] [flink] leletan commented on a diff in pull request #20852: [FLINK-27101][checkpointing][rest] Add restful API to trigger checkpoints

2022-09-30 Thread GitBox


leletan commented on code in PR #20852:
URL: https://github.com/apache/flink/pull/20852#discussion_r984896360


##
flink-core/src/main/java/org/apache/flink/core/execution/CheckpointType.java:
##
@@ -0,0 +1,52 @@
+/*
+ * 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.core.execution;
+
+import org.apache.flink.annotation.Internal;
+import org.apache.flink.annotation.PublicEvolving;
+import org.apache.flink.configuration.DescribedEnum;
+import org.apache.flink.configuration.description.InlineElement;
+
+import static org.apache.flink.configuration.description.TextElement.text;
+
+/** Describes the type in which a checkpoint should be taken. */
+@PublicEvolving
+public enum CheckpointType implements DescribedEnum {
+/** A checkpoint type with the entire state check pointed, common for all 
state backends. */

Review Comment:
   This is copied from 
[SavepointFormatType](https://github.com/apache/flink/blob/f543b8ac690b1dee58bc3cb345a1c8ad0db0941e/flink-core/src/main/java/org/apache/flink/core/execution/SavepointFormatType.java)
   But I am fine with removing it.



##
flink-core/src/main/java/org/apache/flink/core/execution/CheckpointType.java:
##
@@ -0,0 +1,52 @@
+/*
+ * 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.core.execution;
+
+import org.apache.flink.annotation.Internal;
+import org.apache.flink.annotation.PublicEvolving;
+import org.apache.flink.configuration.DescribedEnum;
+import org.apache.flink.configuration.description.InlineElement;
+
+import static org.apache.flink.configuration.description.TextElement.text;
+
+/** Describes the type in which a checkpoint should be taken. */
+@PublicEvolving
+public enum CheckpointType implements DescribedEnum {
+/** A checkpoint type with the entire state check pointed, common for all 
state backends. */
+FULL("A checkpoint type with the entire state check pointed, common for 
all state backends."),
+
+/**
+ * A checkpoint type that check points only the difference between 
snapshots, specific for

Review Comment:
   This is copied from 
[SavepointFormatType](https://github.com/apache/flink/blob/f543b8ac690b1dee58bc3cb345a1c8ad0db0941e/flink-core/src/main/java/org/apache/flink/core/execution/SavepointFormatType.java)
   But I am fine with removing it.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



[GitHub] [flink] flinkbot commented on pull request #20934: Bump pulsar-broker from 2.10.0 to 2.10.1 in /flink-connectors/flink-connector-pulsar

2022-09-30 Thread GitBox


flinkbot commented on PR #20934:
URL: https://github.com/apache/flink/pull/20934#issuecomment-1263891667

   
   ## CI report:
   
   * 6cbcae8c6e692a220051d1361de480c41f106ef0 UNKNOWN
   
   
   Bot commands
 The @flinkbot bot supports the following commands:
   
- `@flinkbot run azure` re-run the last Azure build
   


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



[GitHub] [flink] dependabot[bot] opened a new pull request, #20934: Bump pulsar-broker from 2.10.0 to 2.10.1 in /flink-connectors/flink-connector-pulsar

2022-09-30 Thread GitBox


dependabot[bot] opened a new pull request, #20934:
URL: https://github.com/apache/flink/pull/20934

   Bumps [pulsar-broker](https://github.com/apache/pulsar) from 2.10.0 to 
2.10.1.
   
   Release notes
   Sourced from https://github.com/apache/pulsar/releases";>pulsar-broker's 
releases.
   
   v2.10.1
   Important notice
   
   [fix][broker] Fix broker LoadBalance uneffective https://github-redirect.dependabot.com/apache/pulsar/pull/15314";>15314
   [fix][admin] Fix producer/consume permission can’t get schema https://github-redirect.dependabot.com/apache/pulsar/pull/15956";>15956
   
   Broker
   
   [fix][broker] Fix race condition in getManagedLedgerInternalStats when 
includeLedgerMetadata=true https://github-redirect.dependabot.com/apache/pulsar/pull/15918";>15918
   [improve][broker] Avoid contended synchronized block on topic load https://github-redirect.dependabot.com/apache/pulsar/pull/15883";>15883
   [fix][broker] Fix NPE in MessageDeduplication https://github-redirect.dependabot.com/apache/pulsar/pull/15820";>15820
   [improve][broker] Add timeout to closing CoordinationServiceImpl https://github-redirect.dependabot.com/apache/pulsar/pull/15777";>15777
   [improve][broker] Store Subscription properties https://github-redirect.dependabot.com/apache/pulsar/pull/15757";>15757
   [improve][broker] Support for updating the Subscription properties https://github-redirect.dependabot.com/apache/pulsar/pull/15751";>15751
   [improve][broker] Disable memory limit controller for broker client and 
replication clients https://github-redirect.dependabot.com/apache/pulsar/pull/15723";>15723
   [fix][broker] Fix NPE when put value to RangeCache. https://github-redirect.dependabot.com/apache/pulsar/pull/15707";>15707
   [fix][broker] Fast return if ack cumulative illegal https://github-redirect.dependabot.com/apache/pulsar/pull/15695";>15695
   [fix][broker] Fix creating producer failure when set backlog quota. https://github-redirect.dependabot.com/apache/pulsar/pull/15663";>15663
   [fix][broker] Expose configurationMetadataStore and localMetadataStore 
https://github-redirect.dependabot.com/apache/pulsar/pull/15661";>15661
   [fix][broker] Fix NPE when set AutoTopicCreationOverride https://github-redirect.dependabot.com/apache/pulsar/pull/15653";>15653
   [fix][broker] Fix MetadataStoreException$NotFoundException while doing 
topic lookup https://github-redirect.dependabot.com/apache/pulsar/pull/15633";>15633
   [fix][broker] Fix calculation in getNumberOfEntriesInStorage https://github-redirect.dependabot.com/apache/pulsar/pull/15627";>15627
   [fix][broker] Use dispatchRateLimiterLock to update dispatchRateLimiter 
https://github-redirect.dependabot.com/apache/pulsar/pull/15601";>15601
   [fix][broker] Sync topicPublishRateLimiter update https://github-redirect.dependabot.com/apache/pulsar/pull/15599";>15599
   [fix][broker] Fix deadlock in broker after race condition in topic 
creation failure https://github-redirect.dependabot.com/apache/pulsar/pull/15570";>15570
   [cleanup][broker] Override close method to avoid caching exception. https://github-redirect.dependabot.com/apache/pulsar/pull/15529";>15529
   [fix][broker] Close publishLimiter when disable it https://github-redirect.dependabot.com/apache/pulsar/pull/15520";>15520
   [fix][broker] Fix to avoid TopicStatsImpl NPE even if producerName is 
null https://github-redirect.dependabot.com/apache/pulsar/pull/15502";>15502
   [fix][broker] Fix key-shared delivery of messages with interleaved 
delays https://github-redirect.dependabot.com/apache/pulsar/pull/15409";>15409
   [fix][Broker] Fix bug in contructor of RocksdbMetadataStore https://github-redirect.dependabot.com/apache/pulsar/pull/15405";>15405
   [feature][broker] EntryFilter (PIP-105) - support per-Consumer filtering 
https://github-redirect.dependabot.com/apache/pulsar/pull/15391";>15391
   [fix][broker/client] Close connection if a ping or pong message cannot 
be sent https://github-redirect.dependabot.com/apache/pulsar/pull/15382";>15382
   [improve][broker] Support shrink for ConcurrentSortedLongPairSet  https://github-redirect.dependabot.com/apache/pulsar/pull/15354";>15354
   [improve][broker] Support properties on NonDurable subscriptions 
(PIP-105) https://github-redirect.dependabot.com/apache/pulsar/pull/15345";>15345
   [improve][broker] Use shrink map for message redelivery. https://github-redirect.dependabot.com/apache/pulsar/pull/15342";>15342
   [fix][Broker] Fix error on recycled SubscriptionPropertiesList https://github-redirect.dependabot.com/apache/pulsar/pull/15335";>15335
   [fix][broker] Fix problem at RateLimiter#tryAcquire https://github-redirect.dependabot.com/apache/pulsar/pull/15306";>15306
   [fix][broker] Fix totalEntries calculation problem in 
AbstractBaseDispatcher#filterEntriesForConsumere https://github-redirect.dependabot.com/apache/pulsar/pull/15298";>15298
   [fix][broker] Fix resource group does not report usage https://github-redirect.dependabot.com

[GitHub] [flink] stevenzwu commented on pull request #19537: [FLINK-23589][flink-avro] Support microsecond precision for timestamp

2022-09-30 Thread GitBox


stevenzwu commented on PR #19537:
URL: https://github.com/apache/flink/pull/19537#issuecomment-1263847172

   @matriv is this PR blocked by 
https://issues.apache.org/jira/browse/FLINK-17224 and 
https://issues.apache.org/jira/browse/FLINK-27331? Without them, will the 
converters btw Flink RowData and Avro GenericRecord work with the microsecond 
precision after this fix?


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



[GitHub] [flink-ml] lindong28 commented on a diff in pull request #158: [FLINK-29409] Add Transformer and Estimator for VarianceThresholdSelector

2022-09-30 Thread GitBox


lindong28 commented on code in PR #158:
URL: https://github.com/apache/flink-ml/pull/158#discussion_r984668405


##
flink-ml-lib/src/test/java/org/apache/flink/ml/feature/VarianceThresholdSelectorTest.java:
##
@@ -0,0 +1,294 @@
+/*
+ * 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.ml.feature;
+
+import org.apache.flink.api.common.functions.MapFunction;
+import org.apache.flink.api.common.restartstrategy.RestartStrategies;
+import org.apache.flink.configuration.Configuration;
+import 
org.apache.flink.ml.feature.variancethresholdselector.VarianceThresholdSelector;
+import 
org.apache.flink.ml.feature.variancethresholdselector.VarianceThresholdSelectorModel;
+import org.apache.flink.ml.linalg.DenseVector;
+import org.apache.flink.ml.linalg.SparseVector;
+import org.apache.flink.ml.linalg.Vectors;
+import org.apache.flink.ml.util.TestUtils;
+import org.apache.flink.streaming.api.datastream.DataStream;
+import 
org.apache.flink.streaming.api.environment.ExecutionCheckpointingOptions;
+import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
+import org.apache.flink.table.api.Expressions;
+import org.apache.flink.table.api.Table;
+import org.apache.flink.table.api.bridge.java.StreamTableEnvironment;
+import org.apache.flink.table.api.internal.TableImpl;
+import org.apache.flink.test.util.AbstractTestBase;
+import org.apache.flink.types.Row;
+
+import org.apache.commons.collections.IteratorUtils;
+import org.apache.commons.lang3.exception.ExceptionUtils;
+import org.junit.Before;
+import org.junit.Rule;
+import org.junit.Test;
+import org.junit.rules.TemporaryFolder;
+
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.List;
+
+import static org.junit.Assert.assertArrayEquals;
+import static org.junit.Assert.assertEquals;
+import static org.junit.Assert.assertTrue;
+import static org.junit.Assert.fail;
+
+/** Tests {@link VarianceThresholdSelector} and {@link 
VarianceThresholdSelectorModel}. */
+public class VarianceThresholdSelectorTest extends AbstractTestBase {
+
+@Rule public final TemporaryFolder tempFolder = new TemporaryFolder();
+private StreamExecutionEnvironment env;
+private StreamTableEnvironment tEnv;
+private Table trainDataTable;
+private Table predictDataTable;
+
+private static final double EPS = 1.0e-5;
+private static final List TRAIN_DATA =
+new ArrayList<>(
+Arrays.asList(
+Row.of(1, Vectors.dense(5.0, 7.0, 0.0, 7.0, 6.0, 
0.0)),
+Row.of(2, Vectors.dense(0.0, 9.0, 6.0, 0.0, 5.0, 
9.0)),
+Row.of(3, Vectors.dense(0.0, 9.0, 3.0, 0.0, 5.0, 
5.0)),
+Row.of(4, Vectors.dense(1.0, 9.0, 8.0, 5.0, 7.0, 
4.0)),
+Row.of(5, Vectors.dense(9.0, 8.0, 6.0, 5.0, 4.0, 
4.0)),
+Row.of(6, Vectors.dense(6.0, 9.0, 7.0, 0.0, 2.0, 
0.0;
+
+private static final List PREDICT_DATA =
+new ArrayList<>(
+Arrays.asList(
+Row.of(Vectors.dense(1.0, 2.0, 3.0, 4.0, 5.0, 
6.0)),
+Row.of(Vectors.dense(0.1, 0.2, 0.3, 0.4, 0.5, 
0.6;
+
+private static final List EXPECTED_OUTPUT_TRAIN =

Review Comment:
   Should we remove this?



##
flink-ml-lib/src/test/java/org/apache/flink/ml/feature/VarianceThresholdSelectorTest.java:
##
@@ -0,0 +1,294 @@
+/*
+ * 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.
+ * Se

[jira] [Created] (FLINK-29489) Display issue when querying complex, deeply nested fields

2022-09-30 Thread Justin (Jira)
Justin created FLINK-29489:
--

 Summary: Display issue when querying complex, deeply nested fields
 Key: FLINK-29489
 URL: https://issues.apache.org/jira/browse/FLINK-29489
 Project: Flink
  Issue Type: Bug
Reporter: Justin
 Attachments: FlinkQueries.txt, flink-screenshot1.jpg, 
flink-screenshot2.jpg

Using Flink 1.15, I’m observing some strange behavior when querying fields that 
have complex, deeply nested fields.

The attached file, FlinkQueries.txt, has two create table statements.

In the first table, I create a Flink table on top of the `tfmsStatusOutput` 
field. I have no issues querying this field using the SQL client or Java API.

The second table, I create a Flink table on top of the `fltdOutput` field. This 
field is more complex and has deeply nested fields. Using the SQL client, when 
I run a simple bounded query such as `SELECT * FROM TBL2 LIMIT 1;` I get a 
stack trace dump in my display window and when I press any key on my keyboard 
it returns to the result window as if it's waiting for the results. Those are 
screenshots  (flink-screenshot1 and flink-screenshot2).

 

Using the Java API, I experience something similar. 

It looks like it is going to return something good:

Starting the SELECT...
Job has been submitted with JobID 0b38a2c51e8357e5fef471b38ac839d0
+++++
| op |                fltdOutput |

but then it starts spitting out some form of diagnostics that doesn’t make much 
sense:

/* 1 */public final class GeneratedCastExecutor$45 implements 
org.apache.flink.table.data.utils.CastExecutor {
/* 2 */java.lang.StringBuilder builder$46 = new java.lang.StringBuilder();
/* 3 */java.lang.StringBuilder builder$50 = new java.lang.StringBuilder();
/* 4 */java.lang.StringBuilder builder$55 = new java.lang.StringBuilder();
…
/* 845 */java.lang.StringBuilder builder$7642 = new java.lang.StringBuilder();
/* 846 */java.lang.StringBuilder builder$7654 = new java.lang.StringBuilder();
/* 847 */java.lang.StringBuilder builder$7664 = new java.lang.StringBuilder();
/* 848 */java.lang.StringBuilder builder$7672 = new java.lang.StringBuilder();
/* 849 */public GeneratedCastExecutor$45() {
/* 850 */}
/* 851 */@Override public Object cast(Object _myInputObj) throws 
org.apache.flink.table.api.TableException {
/* 852 */org.apache.flink.table.data.RowData _myInput = 
((org.apache.flink.table.data.RowData)(_myInputObj));
/* 853 */boolean _myInputIsNull = _myInputObj == null;
/* 854 */boolean isNull$0;
/* 855 */org.apache.flink.table.data.binary.BinaryStringData result$1;
/* 856 */boolean isNull$2;
/* 857 */org.apache.flink.table.data.binary.BinaryStringData result$3;
/* 858 */boolean isNull$4;

etc…

 

I couldn't find anything in the task manager logs.



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[GitHub] [flink] alpinegizmo commented on a diff in pull request #20893: [FLINK-29403][cep] Streamline SimpleCondition usage

2022-09-30 Thread GitBox


alpinegizmo commented on code in PR #20893:
URL: https://github.com/apache/flink/pull/20893#discussion_r984666197


##
flink-libraries/flink-cep/src/main/java/org/apache/flink/cep/pattern/conditions/SimpleCondition.java:
##
@@ -40,4 +40,13 @@
 public boolean filter(T value, Context ctx) throws Exception {
 return filter(value);
 }
+
+public static  SimpleCondition of(FilterFunction filters) {

Review Comment:
   This is already a big improvement in usability, and I don't see how to do 
better without introducing a breaking (or confusing) change. 



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



[jira] [Commented] (FLINK-29427) LookupJoinITCase failed with classloader problem

2022-09-30 Thread Huang Xingbo (Jira)


[ 
https://issues.apache.org/jira/browse/FLINK-29427?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17611629#comment-17611629
 ] 

Huang Xingbo commented on FLINK-29427:
--

https://dev.azure.com/apache-flink/apache-flink/_build/results?buildId=41488&view=logs&j=0c940707-2659-5648-cbe6-a1ad63045f0a&t=075c2716-8010-5565-fe08-3c4bb45824a4&l=21930

> LookupJoinITCase failed with classloader problem
> 
>
> Key: FLINK-29427
> URL: https://issues.apache.org/jira/browse/FLINK-29427
> Project: Flink
>  Issue Type: Bug
>  Components: Table SQL / Planner
>Affects Versions: 1.16.0
>Reporter: Huang Xingbo
>Priority: Critical
>  Labels: test-stability
>
> {code:java}
> 2022-09-27T02:49:20.9501313Z Sep 27 02:49:20 Caused by: 
> org.codehaus.janino.InternalCompilerException: Compiling 
> "KeyProjection$108341": Trying to access closed classloader. Please check if 
> you store classloaders directly or indirectly in static fields. If the 
> stacktrace suggests that the leak occurs in a third party library and cannot 
> be fixed immediately, you can disable this check with the configuration 
> 'classloader.check-leaked-classloader'.
> 2022-09-27T02:49:20.9502654Z Sep 27 02:49:20  at 
> org.codehaus.janino.UnitCompiler.compileUnit(UnitCompiler.java:382)
> 2022-09-27T02:49:20.9503366Z Sep 27 02:49:20  at 
> org.codehaus.janino.SimpleCompiler.cook(SimpleCompiler.java:237)
> 2022-09-27T02:49:20.9504044Z Sep 27 02:49:20  at 
> org.codehaus.janino.SimpleCompiler.compileToClassLoader(SimpleCompiler.java:465)
> 2022-09-27T02:49:20.9504704Z Sep 27 02:49:20  at 
> org.codehaus.janino.SimpleCompiler.cook(SimpleCompiler.java:216)
> 2022-09-27T02:49:20.9505341Z Sep 27 02:49:20  at 
> org.codehaus.janino.SimpleCompiler.cook(SimpleCompiler.java:207)
> 2022-09-27T02:49:20.9505965Z Sep 27 02:49:20  at 
> org.codehaus.commons.compiler.Cookable.cook(Cookable.java:80)
> 2022-09-27T02:49:20.9506584Z Sep 27 02:49:20  at 
> org.codehaus.commons.compiler.Cookable.cook(Cookable.java:75)
> 2022-09-27T02:49:20.9507261Z Sep 27 02:49:20  at 
> org.apache.flink.table.runtime.generated.CompileUtils.doCompile(CompileUtils.java:104)
> 2022-09-27T02:49:20.9507883Z Sep 27 02:49:20  ... 30 more
> 2022-09-27T02:49:20.9509266Z Sep 27 02:49:20 Caused by: 
> java.lang.IllegalStateException: Trying to access closed classloader. Please 
> check if you store classloaders directly or indirectly in static fields. If 
> the stacktrace suggests that the leak occurs in a third party library and 
> cannot be fixed immediately, you can disable this check with the 
> configuration 'classloader.check-leaked-classloader'.
> 2022-09-27T02:49:20.9510835Z Sep 27 02:49:20  at 
> org.apache.flink.util.FlinkUserCodeClassLoaders$SafetyNetWrapperClassLoader.ensureInner(FlinkUserCodeClassLoaders.java:184)
> 2022-09-27T02:49:20.9511760Z Sep 27 02:49:20  at 
> org.apache.flink.util.FlinkUserCodeClassLoaders$SafetyNetWrapperClassLoader.loadClass(FlinkUserCodeClassLoaders.java:192)
> 2022-09-27T02:49:20.9512456Z Sep 27 02:49:20  at 
> java.lang.Class.forName0(Native Method)
> 2022-09-27T02:49:20.9513014Z Sep 27 02:49:20  at 
> java.lang.Class.forName(Class.java:348)
> 2022-09-27T02:49:20.9513649Z Sep 27 02:49:20  at 
> org.codehaus.janino.ClassLoaderIClassLoader.findIClass(ClassLoaderIClassLoader.java:89)
> 2022-09-27T02:49:20.9514339Z Sep 27 02:49:20  at 
> org.codehaus.janino.IClassLoader.loadIClass(IClassLoader.java:312)
> 2022-09-27T02:49:20.9514990Z Sep 27 02:49:20  at 
> org.codehaus.janino.UnitCompiler.findTypeByName(UnitCompiler.java:8556)
> 2022-09-27T02:49:20.9515659Z Sep 27 02:49:20  at 
> org.codehaus.janino.UnitCompiler.getReferenceType(UnitCompiler.java:6749)
> 2022-09-27T02:49:20.9516337Z Sep 27 02:49:20  at 
> org.codehaus.janino.UnitCompiler.getReferenceType(UnitCompiler.java:6594)
> 2022-09-27T02:49:20.9516989Z Sep 27 02:49:20  at 
> org.codehaus.janino.UnitCompiler.getType2(UnitCompiler.java:6573)
> 2022-09-27T02:49:20.9517632Z Sep 27 02:49:20  at 
> org.codehaus.janino.UnitCompiler.access$13900(UnitCompiler.java:215)
> 2022-09-27T02:49:20.9518319Z Sep 27 02:49:20  at 
> org.codehaus.janino.UnitCompiler$22$1.visitReferenceType(UnitCompiler.java:6481)
> 2022-09-27T02:49:20.9519018Z Sep 27 02:49:20  at 
> org.codehaus.janino.UnitCompiler$22$1.visitReferenceType(UnitCompiler.java:6476)
> 2022-09-27T02:49:20.9519680Z Sep 27 02:49:20  at 
> org.codehaus.janino.Java$ReferenceType.accept(Java.java:3928)
> 2022-09-27T02:49:20.9520386Z Sep 27 02:49:20  at 
> org.codehaus.janino.UnitCompiler$22.visitType(UnitCompiler.java:6476)
> 2022-09-27T02:49:20.9521042Z Sep 27 02:49:20  at 
> org.codehaus.janino.UnitCompiler$22.visitType(UnitCompiler.java:6469)
> 2022-09-27T02:49:20.9521677Z Sep 27 02:49:20  at 
> org.codehaus.janino.Java$ReferenceType.accept(Java.j

[jira] [Commented] (FLINK-29427) LookupJoinITCase failed with classloader problem

2022-09-30 Thread Huang Xingbo (Jira)


[ 
https://issues.apache.org/jira/browse/FLINK-29427?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17611628#comment-17611628
 ] 

Huang Xingbo commented on FLINK-29427:
--

https://dev.azure.com/apache-flink/apache-flink/_build/results?buildId=41493&view=logs&j=0c940707-2659-5648-cbe6-a1ad63045f0a&t=075c2716-8010-5565-fe08-3c4bb45824a4&l=21545

> LookupJoinITCase failed with classloader problem
> 
>
> Key: FLINK-29427
> URL: https://issues.apache.org/jira/browse/FLINK-29427
> Project: Flink
>  Issue Type: Bug
>  Components: Table SQL / Planner
>Affects Versions: 1.16.0
>Reporter: Huang Xingbo
>Priority: Critical
>  Labels: test-stability
>
> {code:java}
> 2022-09-27T02:49:20.9501313Z Sep 27 02:49:20 Caused by: 
> org.codehaus.janino.InternalCompilerException: Compiling 
> "KeyProjection$108341": Trying to access closed classloader. Please check if 
> you store classloaders directly or indirectly in static fields. If the 
> stacktrace suggests that the leak occurs in a third party library and cannot 
> be fixed immediately, you can disable this check with the configuration 
> 'classloader.check-leaked-classloader'.
> 2022-09-27T02:49:20.9502654Z Sep 27 02:49:20  at 
> org.codehaus.janino.UnitCompiler.compileUnit(UnitCompiler.java:382)
> 2022-09-27T02:49:20.9503366Z Sep 27 02:49:20  at 
> org.codehaus.janino.SimpleCompiler.cook(SimpleCompiler.java:237)
> 2022-09-27T02:49:20.9504044Z Sep 27 02:49:20  at 
> org.codehaus.janino.SimpleCompiler.compileToClassLoader(SimpleCompiler.java:465)
> 2022-09-27T02:49:20.9504704Z Sep 27 02:49:20  at 
> org.codehaus.janino.SimpleCompiler.cook(SimpleCompiler.java:216)
> 2022-09-27T02:49:20.9505341Z Sep 27 02:49:20  at 
> org.codehaus.janino.SimpleCompiler.cook(SimpleCompiler.java:207)
> 2022-09-27T02:49:20.9505965Z Sep 27 02:49:20  at 
> org.codehaus.commons.compiler.Cookable.cook(Cookable.java:80)
> 2022-09-27T02:49:20.9506584Z Sep 27 02:49:20  at 
> org.codehaus.commons.compiler.Cookable.cook(Cookable.java:75)
> 2022-09-27T02:49:20.9507261Z Sep 27 02:49:20  at 
> org.apache.flink.table.runtime.generated.CompileUtils.doCompile(CompileUtils.java:104)
> 2022-09-27T02:49:20.9507883Z Sep 27 02:49:20  ... 30 more
> 2022-09-27T02:49:20.9509266Z Sep 27 02:49:20 Caused by: 
> java.lang.IllegalStateException: Trying to access closed classloader. Please 
> check if you store classloaders directly or indirectly in static fields. If 
> the stacktrace suggests that the leak occurs in a third party library and 
> cannot be fixed immediately, you can disable this check with the 
> configuration 'classloader.check-leaked-classloader'.
> 2022-09-27T02:49:20.9510835Z Sep 27 02:49:20  at 
> org.apache.flink.util.FlinkUserCodeClassLoaders$SafetyNetWrapperClassLoader.ensureInner(FlinkUserCodeClassLoaders.java:184)
> 2022-09-27T02:49:20.9511760Z Sep 27 02:49:20  at 
> org.apache.flink.util.FlinkUserCodeClassLoaders$SafetyNetWrapperClassLoader.loadClass(FlinkUserCodeClassLoaders.java:192)
> 2022-09-27T02:49:20.9512456Z Sep 27 02:49:20  at 
> java.lang.Class.forName0(Native Method)
> 2022-09-27T02:49:20.9513014Z Sep 27 02:49:20  at 
> java.lang.Class.forName(Class.java:348)
> 2022-09-27T02:49:20.9513649Z Sep 27 02:49:20  at 
> org.codehaus.janino.ClassLoaderIClassLoader.findIClass(ClassLoaderIClassLoader.java:89)
> 2022-09-27T02:49:20.9514339Z Sep 27 02:49:20  at 
> org.codehaus.janino.IClassLoader.loadIClass(IClassLoader.java:312)
> 2022-09-27T02:49:20.9514990Z Sep 27 02:49:20  at 
> org.codehaus.janino.UnitCompiler.findTypeByName(UnitCompiler.java:8556)
> 2022-09-27T02:49:20.9515659Z Sep 27 02:49:20  at 
> org.codehaus.janino.UnitCompiler.getReferenceType(UnitCompiler.java:6749)
> 2022-09-27T02:49:20.9516337Z Sep 27 02:49:20  at 
> org.codehaus.janino.UnitCompiler.getReferenceType(UnitCompiler.java:6594)
> 2022-09-27T02:49:20.9516989Z Sep 27 02:49:20  at 
> org.codehaus.janino.UnitCompiler.getType2(UnitCompiler.java:6573)
> 2022-09-27T02:49:20.9517632Z Sep 27 02:49:20  at 
> org.codehaus.janino.UnitCompiler.access$13900(UnitCompiler.java:215)
> 2022-09-27T02:49:20.9518319Z Sep 27 02:49:20  at 
> org.codehaus.janino.UnitCompiler$22$1.visitReferenceType(UnitCompiler.java:6481)
> 2022-09-27T02:49:20.9519018Z Sep 27 02:49:20  at 
> org.codehaus.janino.UnitCompiler$22$1.visitReferenceType(UnitCompiler.java:6476)
> 2022-09-27T02:49:20.9519680Z Sep 27 02:49:20  at 
> org.codehaus.janino.Java$ReferenceType.accept(Java.java:3928)
> 2022-09-27T02:49:20.9520386Z Sep 27 02:49:20  at 
> org.codehaus.janino.UnitCompiler$22.visitType(UnitCompiler.java:6476)
> 2022-09-27T02:49:20.9521042Z Sep 27 02:49:20  at 
> org.codehaus.janino.UnitCompiler$22.visitType(UnitCompiler.java:6469)
> 2022-09-27T02:49:20.9521677Z Sep 27 02:49:20  at 
> org.codehaus.janino.Java$ReferenceType.accept(Java.j

[jira] [Commented] (FLINK-29419) HybridShuffle.testHybridFullExchangesRestart hangs

2022-09-30 Thread Huang Xingbo (Jira)


[ 
https://issues.apache.org/jira/browse/FLINK-29419?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17611627#comment-17611627
 ] 

Huang Xingbo commented on FLINK-29419:
--

https://dev.azure.com/apache-flink/apache-flink/_build/results?buildId=41494&view=logs&j=5c8e7682-d68f-54d1-16a2-a09310218a49&t=86f654fa-ab48-5c1a-25f4-7e7f6afb9bba&l=11782

> HybridShuffle.testHybridFullExchangesRestart hangs
> --
>
> Key: FLINK-29419
> URL: https://issues.apache.org/jira/browse/FLINK-29419
> Project: Flink
>  Issue Type: Bug
>  Components: Runtime / Network
>Affects Versions: 1.16.0, 1.17.0
>Reporter: Huang Xingbo
>Assignee: Weijie Guo
>Priority: Critical
>  Labels: pull-request-available, test-stability
>
> {code:java}
> 2022-09-26T10:56:44.0766792Z Sep 26 10:56:44 "ForkJoinPool-1-worker-25" #27 
> daemon prio=5 os_prio=0 tid=0x7f41a4efa000 nid=0x6d76 waiting on 
> condition [0x7f40ac135000]
> 2022-09-26T10:56:44.0767432Z Sep 26 10:56:44java.lang.Thread.State: 
> WAITING (parking)
> 2022-09-26T10:56:44.0767892Z Sep 26 10:56:44  at sun.misc.Unsafe.park(Native 
> Method)
> 2022-09-26T10:56:44.0768644Z Sep 26 10:56:44  - parking to wait for  
> <0xa0704e18> (a java.util.concurrent.CompletableFuture$Signaller)
> 2022-09-26T10:56:44.0769287Z Sep 26 10:56:44  at 
> java.util.concurrent.locks.LockSupport.park(LockSupport.java:175)
> 2022-09-26T10:56:44.0769949Z Sep 26 10:56:44  at 
> java.util.concurrent.CompletableFuture$Signaller.block(CompletableFuture.java:1707)
> 2022-09-26T10:56:44.0770623Z Sep 26 10:56:44  at 
> java.util.concurrent.ForkJoinPool.managedBlock(ForkJoinPool.java:3313)
> 2022-09-26T10:56:44.0771349Z Sep 26 10:56:44  at 
> java.util.concurrent.CompletableFuture.waitingGet(CompletableFuture.java:1742)
> 2022-09-26T10:56:44.0772092Z Sep 26 10:56:44  at 
> java.util.concurrent.CompletableFuture.get(CompletableFuture.java:1908)
> 2022-09-26T10:56:44.0772777Z Sep 26 10:56:44  at 
> org.apache.flink.test.runtime.JobGraphRunningUtil.execute(JobGraphRunningUtil.java:57)
> 2022-09-26T10:56:44.0773534Z Sep 26 10:56:44  at 
> org.apache.flink.test.runtime.BatchShuffleITCaseBase.executeJob(BatchShuffleITCaseBase.java:115)
> 2022-09-26T10:56:44.0774333Z Sep 26 10:56:44  at 
> org.apache.flink.test.runtime.HybridShuffleITCase.testHybridFullExchangesRestart(HybridShuffleITCase.java:59)
> {code}
> https://dev.azure.com/apache-flink/apache-flink/_build/results?buildId=41343&view=logs&j=a57e0635-3fad-5b08-57c7-a4142d7d6fa9&t=2ef0effc-1da1-50e5-c2bd-aab434b1c5b7



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Commented] (FLINK-29483) flink python udf arrow in thread model bug

2022-09-30 Thread Huang Xingbo (Jira)


[ 
https://issues.apache.org/jira/browse/FLINK-29483?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17611617#comment-17611617
 ] 

Huang Xingbo commented on FLINK-29483:
--

[~jackylau] You means your job runs in `thread mode` will raise this exception? 

> flink python udf arrow in thread model bug
> --
>
> Key: FLINK-29483
> URL: https://issues.apache.org/jira/browse/FLINK-29483
> Project: Flink
>  Issue Type: Bug
>  Components: API / Python
>Affects Versions: 1.16.0, 1.15.2
>Reporter: jackylau
>Priority: Major
> Fix For: 1.17.0, 1.15.3, 1.16.1
>
> Attachments: image-2022-09-30-17-03-05-005.png
>
>
> !image-2022-09-30-17-03-05-005.png!



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[GitHub] [flink] zentol commented on a diff in pull request #20930: [FLINK-29455] Introduce OperatorIdentifier

2022-09-30 Thread GitBox


zentol commented on code in PR #20930:
URL: https://github.com/apache/flink/pull/20930#discussion_r984616865


##
flink-libraries/flink-state-processing-api/src/main/java/org/apache/flink/state/api/OperatorIdentifier.java:
##
@@ -0,0 +1,75 @@
+/*
+ * 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.state.api;
+
+import org.apache.flink.annotation.Internal;
+import org.apache.flink.runtime.jobgraph.OperatorID;
+import org.apache.flink.state.api.runtime.OperatorIDGenerator;
+import org.apache.flink.util.Preconditions;
+import org.apache.flink.util.StringUtils;
+
+import javax.annotation.Nullable;
+
+import java.io.Serializable;
+import java.util.Objects;
+import java.util.Optional;
+
+/** Identifies an operator, either based on a {@code uid} or {@code uidHash}. 
*/
+@Internal
+public class OperatorIdentifier implements Serializable {
+private final OperatorID operatorId;

Review Comment:
   I'll re-order them and add a clarifying comment.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



[GitHub] [flink] alpinegizmo commented on a diff in pull request #20930: [FLINK-29455] Introduce OperatorIdentifier

2022-09-30 Thread GitBox


alpinegizmo commented on code in PR #20930:
URL: https://github.com/apache/flink/pull/20930#discussion_r984612707


##
flink-libraries/flink-state-processing-api/src/main/java/org/apache/flink/state/api/OperatorIdentifier.java:
##
@@ -0,0 +1,75 @@
+/*
+ * 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.state.api;
+
+import org.apache.flink.annotation.Internal;
+import org.apache.flink.runtime.jobgraph.OperatorID;
+import org.apache.flink.state.api.runtime.OperatorIDGenerator;
+import org.apache.flink.util.Preconditions;
+import org.apache.flink.util.StringUtils;
+
+import javax.annotation.Nullable;
+
+import java.io.Serializable;
+import java.util.Objects;
+import java.util.Optional;
+
+/** Identifies an operator, either based on a {@code uid} or {@code uidHash}. 
*/
+@Internal
+public class OperatorIdentifier implements Serializable {
+private final OperatorID operatorId;

Review Comment:
   Maybe add a comment here explaining that the operatorID is the hashed ID? 
Because I'm not familiar with the OperatorID class, it took me a minute to 
figure out which id is which here. Or it might be enough to re-order the 
appearance of uid and uidHash in the comment above -- having them in the 
opposite order gave me pause as I tried to figure out which of operatorId and 
uid is the uidHash.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



[GitHub] [flink] XComp commented on a diff in pull request #20805: [FLINK-29198][test] Fail after maximum RetryOnException

2022-09-30 Thread GitBox


XComp commented on code in PR #20805:
URL: https://github.com/apache/flink/pull/20805#discussion_r982266603


##
flink-test-utils-parent/flink-test-utils-junit/src/test/java/org/apache/flink/testutils/junit/RetryOnExceptionExtensionTest.java:
##
@@ -42,10 +51,10 @@ class RetryOnExceptionExtensionTest {
 
 @AfterAll
 static void verify() {

Review Comment:
   This test implementation is odd. It doesn't allow the execution of 
individual tests. I've created ~FLINK-29198~ 
[FLINK-29452](https://issues.apache.org/jira/browse/FLINK-29452) to cover that 
topic.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



[GitHub] [flink] RyanSkraba commented on a diff in pull request #20805: [FLINK-29198][test] Fail after maximum RetryOnException

2022-09-30 Thread GitBox


RyanSkraba commented on code in PR #20805:
URL: https://github.com/apache/flink/pull/20805#discussion_r984598630


##
flink-test-utils-parent/flink-test-utils-junit/src/test/java/org/apache/flink/testutils/junit/RetryOnExceptionExtensionTest.java:
##
@@ -42,10 +51,10 @@ class RetryOnExceptionExtensionTest {
 
 @AfterAll
 static void verify() {

Review Comment:
   :D  Can you confirm the JIRA number?



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



[GitHub] [flink] RyanSkraba commented on a diff in pull request #20805: [FLINK-29198][test] Fail after maximum RetryOnException

2022-09-30 Thread GitBox


RyanSkraba commented on code in PR #20805:
URL: https://github.com/apache/flink/pull/20805#discussion_r984596066


##
flink-test-utils-parent/flink-test-utils-junit/src/main/java/org/apache/flink/testutils/junit/extensions/retry/strategy/RetryOnExceptionStrategy.java:
##
@@ -37,6 +40,12 @@ public RetryOnExceptionStrategy(
 @Override
 public void handleException(String testName, int attemptIndex, Throwable 
throwable)
 throws Throwable {
+// Failed when reach the total retry times
+if (attemptIndex >= totalTimes) {
+LOG.error("Test Failed at the last retry.", throwable);
+throw throwable;
+}

Review Comment:
   Hello!  I thought about this too -- the way the code is expressed, there 
could be other implementations of `AbstractRetryStrategy` that apply different 
strategies (for example, to continue retrying for max period of time regardless 
of the number of attempts).
   
   My opinion isn't strong on this, but the logic isn't really complicated 
enough to need to be abstracted away.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



[jira] [Closed] (FLINK-29484) Support orderless check of elements in SourceTestSuiteBase and SinkTestSuiteBase

2022-09-30 Thread Etienne Chauchot (Jira)


 [ 
https://issues.apache.org/jira/browse/FLINK-29484?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Etienne Chauchot closed FLINK-29484.

Resolution: Won't Fix

Closing because it is already available by overriding 
SourceTestSuiteBase#checkResultWithSemantic() and using 
CollectIteratorAssertions#assertUnordered()

> Support orderless check of elements in SourceTestSuiteBase and 
> SinkTestSuiteBase
> 
>
> Key: FLINK-29484
> URL: https://issues.apache.org/jira/browse/FLINK-29484
> Project: Flink
>  Issue Type: Improvement
>  Components: Tests
>Reporter: Etienne Chauchot
>Assignee: Etienne Chauchot
>Priority: Major
>
> Some backend source and sinks are distributed and to not guarantee order. So 
> the test suites must consider that the tests pass even if the data is out of 
> order. We should add a configuration to the ExternalContext to set orderless 
> per source/sink.



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Commented] (FLINK-29485) TypeSerializerUpgradeTestBase still uses 1.15 as the current version on master and release-1.16

2022-09-30 Thread Matthias Pohl (Jira)


[ 
https://issues.apache.org/jira/browse/FLINK-29485?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17611580#comment-17611580
 ] 

Matthias Pohl commented on FLINK-29485:
---

A similar issue exists for {{StatefulJobSnapshotMigrationITCase}}. But there 
it's state that we should only progress on the release branch (i.e. to 1.16). 
Hence, I removed the {{1.17.0}} version tag.

> TypeSerializerUpgradeTestBase still uses 1.15 as the current version on 
> master and release-1.16
> ---
>
> Key: FLINK-29485
> URL: https://issues.apache.org/jira/browse/FLINK-29485
> Project: Flink
>  Issue Type: Bug
>  Components: Tests
>Affects Versions: 1.16.0
>Reporter: Matthias Pohl
>Assignee: Matthias Pohl
>Priority: Blocker
>  Labels: starter
>
> {{TypeSerializerUpgradeTestBase}} still refers to 1.15 as the current 
> version. We could use {{FlinkVersions.current()}} instead to avoid running 
> into this issue again for future major updates.
> I didn't check other occurrences of {{FlinkVersions}}. It should be verified 
> as part of this Jira issue that we don't have the same issue in other 
> locations as well.



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Assigned] (FLINK-29485) TypeSerializerUpgradeTestBase still uses 1.15 as the current version on master and release-1.16

2022-09-30 Thread Matthias Pohl (Jira)


 [ 
https://issues.apache.org/jira/browse/FLINK-29485?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Matthias Pohl reassigned FLINK-29485:
-

Assignee: Matthias Pohl

> TypeSerializerUpgradeTestBase still uses 1.15 as the current version on 
> master and release-1.16
> ---
>
> Key: FLINK-29485
> URL: https://issues.apache.org/jira/browse/FLINK-29485
> Project: Flink
>  Issue Type: Bug
>  Components: Tests
>Affects Versions: 1.16.0, 1.17.0
>Reporter: Matthias Pohl
>Assignee: Matthias Pohl
>Priority: Blocker
>  Labels: starter
>
> {{TypeSerializerUpgradeTestBase}} still refers to 1.15 as the current 
> version. We could use {{FlinkVersions.current()}} instead to avoid running 
> into this issue again for future major updates.
> I didn't check other occurrences of {{FlinkVersions}}. It should be verified 
> as part of this Jira issue that we don't have the same issue in other 
> locations as well.



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Updated] (FLINK-29485) TypeSerializerUpgradeTestBase still uses 1.15 as the current version on master and release-1.16

2022-09-30 Thread Matthias Pohl (Jira)


 [ 
https://issues.apache.org/jira/browse/FLINK-29485?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Matthias Pohl updated FLINK-29485:
--
Affects Version/s: (was: 1.17.0)

> TypeSerializerUpgradeTestBase still uses 1.15 as the current version on 
> master and release-1.16
> ---
>
> Key: FLINK-29485
> URL: https://issues.apache.org/jira/browse/FLINK-29485
> Project: Flink
>  Issue Type: Bug
>  Components: Tests
>Affects Versions: 1.16.0
>Reporter: Matthias Pohl
>Assignee: Matthias Pohl
>Priority: Blocker
>  Labels: starter
>
> {{TypeSerializerUpgradeTestBase}} still refers to 1.15 as the current 
> version. We could use {{FlinkVersions.current()}} instead to avoid running 
> into this issue again for future major updates.
> I didn't check other occurrences of {{FlinkVersions}}. It should be verified 
> as part of this Jira issue that we don't have the same issue in other 
> locations as well.



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Commented] (FLINK-29484) Support orderless check of elements in SourceTestSuiteBase and SinkTestSuiteBase

2022-09-30 Thread Etienne Chauchot (Jira)


[ 
https://issues.apache.org/jira/browse/FLINK-29484?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17611578#comment-17611578
 ] 

Etienne Chauchot commented on FLINK-29484:
--

It seems _UnorderedCollectIteratorAssert_ already exists. Need to leverage it 
in the test suites.

> Support orderless check of elements in SourceTestSuiteBase and 
> SinkTestSuiteBase
> 
>
> Key: FLINK-29484
> URL: https://issues.apache.org/jira/browse/FLINK-29484
> Project: Flink
>  Issue Type: Improvement
>  Components: Tests
>Reporter: Etienne Chauchot
>Assignee: Etienne Chauchot
>Priority: Major
>
> Some backend source and sinks are distributed and to not guarantee order. So 
> the test suites must consider that the tests pass even if the data is out of 
> order. We should add a configuration to the ExternalContext to set orderless 
> per source/sink.



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Updated] (FLINK-29485) TypeSerializerUpgradeTestBase still uses 1.15 as the current version on master and release-1.16

2022-09-30 Thread Matthias Pohl (Jira)


 [ 
https://issues.apache.org/jira/browse/FLINK-29485?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Matthias Pohl updated FLINK-29485:
--
Priority: Blocker  (was: Critical)

> TypeSerializerUpgradeTestBase still uses 1.15 as the current version on 
> master and release-1.16
> ---
>
> Key: FLINK-29485
> URL: https://issues.apache.org/jira/browse/FLINK-29485
> Project: Flink
>  Issue Type: Bug
>  Components: Tests
>Affects Versions: 1.16.0, 1.17.0
>Reporter: Matthias Pohl
>Priority: Blocker
>  Labels: starter
>
> {{TypeSerializerUpgradeTestBase}} still refers to 1.15 as the current 
> version. We could use {{FlinkVersions.current()}} instead to avoid running 
> into this issue again for future major updates.
> I didn't check other occurrences of {{FlinkVersions}}. It should be verified 
> as part of this Jira issue that we don't have the same issue in other 
> locations as well.



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Updated] (FLINK-29485) TypeSerializerUpgradeTestBase still uses 1.15 as the current version on master and release-1.16

2022-09-30 Thread Matthias Pohl (Jira)


 [ 
https://issues.apache.org/jira/browse/FLINK-29485?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Matthias Pohl updated FLINK-29485:
--
Summary: TypeSerializerUpgradeTestBase still uses 1.15 as the current 
version on master and release-1.16  (was: TypeSerializerUpgradeTestBase doesn't 
use FlinkVersions.current())

> TypeSerializerUpgradeTestBase still uses 1.15 as the current version on 
> master and release-1.16
> ---
>
> Key: FLINK-29485
> URL: https://issues.apache.org/jira/browse/FLINK-29485
> Project: Flink
>  Issue Type: Bug
>  Components: Tests
>Affects Versions: 1.16.0, 1.17.0
>Reporter: Matthias Pohl
>Priority: Critical
>  Labels: starter
>
> {{TypeSerializerUpgradeTestBase}} still refers to 1.15 as the current 
> version. We could use {{FlinkVersions.current()}} instead to avoid running 
> into this issue again for future major updates.
> I didn't check other occurrences of {{FlinkVersions}}. It should be verified 
> as part of this Jira issue that we don't have the same issue in other 
> locations as well.



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Comment Edited] (FLINK-29485) TypeSerializerUpgradeTestBase doesn't use FlinkVersions.current()

2022-09-30 Thread Matthias Pohl (Jira)


[ 
https://issues.apache.org/jira/browse/FLINK-29485?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17611564#comment-17611564
 ] 

Matthias Pohl edited comment on FLINK-29485 at 9/30/22 12:45 PM:
-

Using {{FlinkVerions.current()}} doesn't resolve the problem that this test 
still requires some snapshot data to be generated. thanks for pointing that out 
[~snuyanzin]. I updated the title accordingly.


was (Author: mapohl):
Using {{FlinkVerions.current()}} doesn't resolve the problem that this test 
still requires some snapshot data to be generated. thanks for pointing that out 
[~snuyanzin]

> TypeSerializerUpgradeTestBase doesn't use FlinkVersions.current()
> -
>
> Key: FLINK-29485
> URL: https://issues.apache.org/jira/browse/FLINK-29485
> Project: Flink
>  Issue Type: Bug
>  Components: Tests
>Affects Versions: 1.16.0, 1.17.0
>Reporter: Matthias Pohl
>Priority: Critical
>  Labels: starter
>
> {{TypeSerializerUpgradeTestBase}} still refers to 1.15 as the current 
> version. We could use {{FlinkVersions.current()}} instead to avoid running 
> into this issue again for future major updates.
> I didn't check other occurrences of {{FlinkVersions}}. It should be verified 
> as part of this Jira issue that we don't have the same issue in other 
> locations as well.



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Commented] (FLINK-29485) TypeSerializerUpgradeTestBase doesn't use FlinkVersions.current()

2022-09-30 Thread Matthias Pohl (Jira)


[ 
https://issues.apache.org/jira/browse/FLINK-29485?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17611564#comment-17611564
 ] 

Matthias Pohl commented on FLINK-29485:
---

Using {{FlinkVerions.current()}} doesn't resolve the problem that this test 
still requires some snapshot data to be generated. thanks for pointing that out 
[~snuyanzin]

> TypeSerializerUpgradeTestBase doesn't use FlinkVersions.current()
> -
>
> Key: FLINK-29485
> URL: https://issues.apache.org/jira/browse/FLINK-29485
> Project: Flink
>  Issue Type: Bug
>  Components: Tests
>Affects Versions: 1.16.0, 1.17.0
>Reporter: Matthias Pohl
>Priority: Critical
>  Labels: starter
>
> {{TypeSerializerUpgradeTestBase}} still refers to 1.15 as the current 
> version. We could use {{FlinkVersions.current()}} instead to avoid running 
> into this issue again for future major updates.
> I didn't check other occurrences of {{FlinkVersions}}. It should be verified 
> as part of this Jira issue that we don't have the same issue in other 
> locations as well.



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Updated] (FLINK-29476) Kinesis Connector retry mechanism not applied to EOFException

2022-09-30 Thread Danny Cranmer (Jira)


 [ 
https://issues.apache.org/jira/browse/FLINK-29476?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Danny Cranmer updated FLINK-29476:
--
Fix Version/s: 1.16.1

> Kinesis Connector retry mechanism not applied to EOFException
> -
>
> Key: FLINK-29476
> URL: https://issues.apache.org/jira/browse/FLINK-29476
> Project: Flink
>  Issue Type: Improvement
>  Components: Connectors / Kinesis
>Affects Versions: 1.15.2
>Reporter: Alexander Fedulov
>Assignee: Alexander Fedulov
>Priority: Major
>  Labels: pull-request-available
> Fix For: 1.17.0, 1.16.1
>
> Attachments: kinesis-exception.log
>
>
> The current retry mechanism in Kinesis connector only considers 
> _SocketTimeoutException_ as recoverable: 
> [KinesisProxy.java#L422|https://github.com/apache/flink/blob/release-1.16.0-rc1/flink-connectors/flink-connector-kinesis/src/main/java/org/apache/flink/streaming/connectors/kinesis/proxy/KinesisProxy.java#L422]
>  , however we observed that communication can also fail with EOFException: 
> [^kinesis-exception.log]
> This exception should also be considered recoverable and retried.



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Updated] (FLINK-29476) Kinesis Connector retry mechanism not applied to EOFException

2022-09-30 Thread Danny Cranmer (Jira)


 [ 
https://issues.apache.org/jira/browse/FLINK-29476?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Danny Cranmer updated FLINK-29476:
--
Fix Version/s: 1.17.0

> Kinesis Connector retry mechanism not applied to EOFException
> -
>
> Key: FLINK-29476
> URL: https://issues.apache.org/jira/browse/FLINK-29476
> Project: Flink
>  Issue Type: Improvement
>  Components: Connectors / Kinesis
>Affects Versions: 1.15.2
>Reporter: Alexander Fedulov
>Assignee: Alexander Fedulov
>Priority: Major
>  Labels: pull-request-available
> Fix For: 1.17.0
>
> Attachments: kinesis-exception.log
>
>
> The current retry mechanism in Kinesis connector only considers 
> _SocketTimeoutException_ as recoverable: 
> [KinesisProxy.java#L422|https://github.com/apache/flink/blob/release-1.16.0-rc1/flink-connectors/flink-connector-kinesis/src/main/java/org/apache/flink/streaming/connectors/kinesis/proxy/KinesisProxy.java#L422]
>  , however we observed that communication can also fail with EOFException: 
> [^kinesis-exception.log]
> This exception should also be considered recoverable and retried.



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Updated] (FLINK-29476) Kinesis Connector retry mechanism not applied to EOFException

2022-09-30 Thread Danny Cranmer (Jira)


 [ 
https://issues.apache.org/jira/browse/FLINK-29476?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Danny Cranmer updated FLINK-29476:
--
Fix Version/s: 1.15.3

> Kinesis Connector retry mechanism not applied to EOFException
> -
>
> Key: FLINK-29476
> URL: https://issues.apache.org/jira/browse/FLINK-29476
> Project: Flink
>  Issue Type: Improvement
>  Components: Connectors / Kinesis
>Affects Versions: 1.15.2
>Reporter: Alexander Fedulov
>Assignee: Alexander Fedulov
>Priority: Major
>  Labels: pull-request-available
> Fix For: 1.17.0, 1.15.3, 1.16.1
>
> Attachments: kinesis-exception.log
>
>
> The current retry mechanism in Kinesis connector only considers 
> _SocketTimeoutException_ as recoverable: 
> [KinesisProxy.java#L422|https://github.com/apache/flink/blob/release-1.16.0-rc1/flink-connectors/flink-connector-kinesis/src/main/java/org/apache/flink/streaming/connectors/kinesis/proxy/KinesisProxy.java#L422]
>  , however we observed that communication can also fail with EOFException: 
> [^kinesis-exception.log]
> This exception should also be considered recoverable and retried.



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[GitHub] [flink] dannycranmer commented on pull request #20927: [FLINK-29476][connector][kinesis] Retry connection termination with EOFException

2022-09-30 Thread GitBox


dannycranmer commented on PR #20927:
URL: https://github.com/apache/flink/pull/20927#issuecomment-1263505523

   Thanks @afedulov, LGTM.
   
   Can you please also open a PR against `release-1.16` and `release-1.15`?


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



[GitHub] [flink] dannycranmer commented on pull request #20927: [FLINK-29476][connector][kinesis] Retry connection termination with EOFException

2022-09-30 Thread GitBox


dannycranmer commented on PR #20927:
URL: https://github.com/apache/flink/pull/20927#issuecomment-1263505008

   @flinkbot run azure


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



[GitHub] [flink] XComp commented on a diff in pull request #19780: [FLINK-27727][tests] Migrate TypeSerializerUpgradeTestBase to JUnit5

2022-09-30 Thread GitBox


XComp commented on code in PR #19780:
URL: https://github.com/apache/flink/pull/19780#discussion_r983718553


##
flink-core/src/test/java/org/apache/flink/api/common/typeutils/TypeSerializerUpgradeTestBase.java:
##
@@ -24,50 +24,52 @@
 import org.apache.flink.core.memory.DataInputView;
 import org.apache.flink.core.memory.DataOutputSerializer;
 import org.apache.flink.core.memory.DataOutputView;
-import org.apache.flink.util.TestLogger;
+import org.apache.flink.util.TestLoggerExtension;
 
+import org.assertj.core.api.HamcrestCondition;
 import org.hamcrest.Matcher;
-import org.junit.Ignore;
-import org.junit.Test;
+import org.junit.jupiter.api.Disabled;
+import org.junit.jupiter.api.TestInstance;
+import org.junit.jupiter.api.extension.ExtendWith;
+import org.junit.jupiter.params.ParameterizedTest;
+import org.junit.jupiter.params.provider.MethodSource;
 
 import java.io.IOException;
 import java.nio.file.Files;
 import java.nio.file.Path;
 import java.nio.file.Paths;
+import java.util.Collection;
 import java.util.Collections;
 import java.util.List;
 import java.util.Set;
 
 import static org.apache.flink.util.Preconditions.checkNotNull;
+import static org.assertj.core.api.AssertionsForInterfaceTypes.assertThat;
+import static org.assertj.core.api.Assumptions.assumeThat;
 import static org.hamcrest.CoreMatchers.not;
-import static org.junit.Assert.assertThat;
-import static org.junit.Assume.assumeThat;
 
 /**
  * A test base for testing {@link TypeSerializer} upgrades.
  *
- * You can run {@link #generateTestSetupFiles()} on a Flink branch to 
(re-)generate the test data
- * files.
+ * You can run {@link
+ * 
#generateTestSetupFiles(org.apache.flink.api.common.typeutils.TypeSerializerUpgradeTestBase.TestSpecification)}
+ * on a Flink branch to (re-)generate the test data files.
  */
-public abstract class TypeSerializerUpgradeTestBase
-extends TestLogger {
+@ExtendWith(TestLoggerExtension.class)

Review Comment:
   Using the `META-INF/services/org.junit.jupiter.api.extension.Extension` is 
the better approach here.



##
flink-core/src/test/java/org/apache/flink/api/java/typeutils/runtime/CopyableSerializerUpgradeTest.java:
##
@@ -168,8 +159,8 @@ public Matcher testDataMatcher() {
 }
 
 @Test
-public void testF() {
+void testF() {

Review Comment:
   ```suggestion
   void testSimpleCopyableEqualsImplementation() {
   ```
   Can we make this test method more descriptive? (this should be a separate 
hotfix commit)



##
flink-scala/src/test/scala/org/apache/flink/api/scala/typeutils/TraversableSerializerUpgradeTest.scala:
##
@@ -23,58 +23,21 @@ import org.apache.flink.api.common.typeinfo.TypeInformation
 import org.apache.flink.api.common.typeutils.{TypeSerializer, 
TypeSerializerMatchers, TypeSerializerSchemaCompatibility, 
TypeSerializerUpgradeTestBase}
 import 
org.apache.flink.api.common.typeutils.TypeSerializerUpgradeTestBase.TestSpecification
 import org.apache.flink.api.scala.createTypeInformation
+import org.apache.flink.api.scala.typeutils.TraversableSerializerUpgradeTest._
+import 
org.apache.flink.api.scala.typeutils.TraversableSerializerUpgradeTest.Types.Pojo
 
 import org.hamcrest.Matcher
 import org.hamcrest.Matchers.is
-import org.junit.runner.RunWith
-import org.junit.runners.Parameterized
 
 import java.util
 import java.util.function.Supplier
 
 import scala.collection.{mutable, BitSet, LinearSeq}
 
 /** A [[TypeSerializerUpgradeTestBase]] for [[TraversableSerializer]]. */
-@RunWith(classOf[Parameterized])
-class TraversableSerializerUpgradeTest(
-testSpecification: TypeSerializerUpgradeTestBase.TestSpecification[
-  TraversableOnce[_],
-  TraversableOnce[_]])
-  extends TypeSerializerUpgradeTestBase[TraversableOnce[_], 
TraversableOnce[_]](testSpecification)
-
-object TraversableSerializerUpgradeTest {
-
-  object Types {
-
-class Pojo(var name: String, var count: Int) {
-  def this() = this("", -1)
-
-  override def equals(other: Any): Boolean = {
-other match {
-  case oP: Pojo => name == oP.name && count == oP.count
-  case _ => false
-}
-  }
-}
-
-val seqTypeInfo = implicitly[TypeInformation[Seq[Int]]]
-val indexedSeqTypeInfo =
-  implicitly[TypeInformation[IndexedSeq[Int]]]
-val linearSeqTypeInfo = implicitly[TypeInformation[LinearSeq[Int]]]
-val mapTypeInfo = implicitly[TypeInformation[Map[String, Int]]]
-val setTypeInfo = implicitly[TypeInformation[Set[Int]]]
-val bitsetTypeInfo = implicitly[TypeInformation[BitSet]]
-val mutableListTypeInfo =
-  implicitly[TypeInformation[mutable.MutableList[Int]]]
-val seqTupleTypeInfo = implicitly[TypeInformation[Seq[(Int, String)]]]
-val seqPojoTypeInfo = implicitly[TypeInformation[Seq[Pojo]]]
-  }
-
-  import Types._
-
-  @Parameterized.Parameters(name = "Test Specification = {0}")
-  def testSpecifications: util.Collection[TestSpecification[_, _]] = {
-
+class

[jira] [Commented] (FLINK-29387) IntervalJoinITCase.testIntervalJoinSideOutputRightLateData failed with AssertionError

2022-09-30 Thread Chesnay Schepler (Jira)


[ 
https://issues.apache.org/jira/browse/FLINK-29387?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17611543#comment-17611543
 ] 

Chesnay Schepler commented on FLINK-29387:
--

Running the test multiple times in parallel locally makes this pretty easy to 
reproduce.

> IntervalJoinITCase.testIntervalJoinSideOutputRightLateData failed with 
> AssertionError
> -
>
> Key: FLINK-29387
> URL: https://issues.apache.org/jira/browse/FLINK-29387
> Project: Flink
>  Issue Type: Bug
>  Components: API / DataStream
>Affects Versions: 1.17.0
>Reporter: Huang Xingbo
>Priority: Critical
>  Labels: test-stability
>
> {code:java}
> 2022-09-22T04:40:21.9296331Z Sep 22 04:40:21 [ERROR] 
> org.apache.flink.test.streaming.runtime.IntervalJoinITCase.testIntervalJoinSideOutputRightLateData
>   Time elapsed: 2.46 s  <<< FAILURE!
> 2022-09-22T04:40:21.9297487Z Sep 22 04:40:21 java.lang.AssertionError: 
> expected:<[(key,2)]> but was:<[]>
> 2022-09-22T04:40:21.9298208Z Sep 22 04:40:21  at 
> org.junit.Assert.fail(Assert.java:89)
> 2022-09-22T04:40:21.9298927Z Sep 22 04:40:21  at 
> org.junit.Assert.failNotEquals(Assert.java:835)
> 2022-09-22T04:40:21.9299655Z Sep 22 04:40:21  at 
> org.junit.Assert.assertEquals(Assert.java:120)
> 2022-09-22T04:40:21.9300403Z Sep 22 04:40:21  at 
> org.junit.Assert.assertEquals(Assert.java:146)
> 2022-09-22T04:40:21.9301538Z Sep 22 04:40:21  at 
> org.apache.flink.test.streaming.runtime.IntervalJoinITCase.expectInAnyOrder(IntervalJoinITCase.java:521)
> 2022-09-22T04:40:21.9302578Z Sep 22 04:40:21  at 
> org.apache.flink.test.streaming.runtime.IntervalJoinITCase.testIntervalJoinSideOutputRightLateData(IntervalJoinITCase.java:280)
> 2022-09-22T04:40:21.9303641Z Sep 22 04:40:21  at 
> sun.reflect.NativeMethodAccessorImpl.invoke0(Native Method)
> 2022-09-22T04:40:21.9304472Z Sep 22 04:40:21  at 
> sun.reflect.NativeMethodAccessorImpl.invoke(NativeMethodAccessorImpl.java:62)
> 2022-09-22T04:40:21.9305371Z Sep 22 04:40:21  at 
> sun.reflect.DelegatingMethodAccessorImpl.invoke(DelegatingMethodAccessorImpl.java:43)
> 2022-09-22T04:40:21.9306195Z Sep 22 04:40:21  at 
> java.lang.reflect.Method.invoke(Method.java:498)
> 2022-09-22T04:40:21.9307011Z Sep 22 04:40:21  at 
> org.junit.runners.model.FrameworkMethod$1.runReflectiveCall(FrameworkMethod.java:59)
> 2022-09-22T04:40:21.9308077Z Sep 22 04:40:21  at 
> org.junit.internal.runners.model.ReflectiveCallable.run(ReflectiveCallable.java:12)
> 2022-09-22T04:40:21.9308968Z Sep 22 04:40:21  at 
> org.junit.runners.model.FrameworkMethod.invokeExplosively(FrameworkMethod.java:56)
> 2022-09-22T04:40:21.9309849Z Sep 22 04:40:21  at 
> org.junit.internal.runners.statements.InvokeMethod.evaluate(InvokeMethod.java:17)
> 2022-09-22T04:40:21.9310704Z Sep 22 04:40:21  at 
> org.junit.internal.runners.statements.RunBefores.evaluate(RunBefores.java:26)
> 2022-09-22T04:40:21.9311533Z Sep 22 04:40:21  at 
> org.junit.runners.ParentRunner$3.evaluate(ParentRunner.java:306)
> 2022-09-22T04:40:21.9312386Z Sep 22 04:40:21  at 
> org.junit.runners.BlockJUnit4ClassRunner$1.evaluate(BlockJUnit4ClassRunner.java:100)
> 2022-09-22T04:40:21.9313231Z Sep 22 04:40:21  at 
> org.junit.runners.ParentRunner.runLeaf(ParentRunner.java:366)
> 2022-09-22T04:40:21.9314985Z Sep 22 04:40:21  at 
> org.junit.runners.BlockJUnit4ClassRunner.runChild(BlockJUnit4ClassRunner.java:103)
> 2022-09-22T04:40:21.9315857Z Sep 22 04:40:21  at 
> org.junit.runners.BlockJUnit4ClassRunner.runChild(BlockJUnit4ClassRunner.java:63)
> 2022-09-22T04:40:21.9316633Z Sep 22 04:40:21  at 
> org.junit.runners.ParentRunner$4.run(ParentRunner.java:331)
> 2022-09-22T04:40:21.9317450Z Sep 22 04:40:21  at 
> org.junit.runners.ParentRunner$1.schedule(ParentRunner.java:79)
> 2022-09-22T04:40:21.9318209Z Sep 22 04:40:21  at 
> org.junit.runners.ParentRunner.runChildren(ParentRunner.java:329)
> 2022-09-22T04:40:21.9318949Z Sep 22 04:40:21  at 
> org.junit.runners.ParentRunner.access$100(ParentRunner.java:66)
> 2022-09-22T04:40:21.9319680Z Sep 22 04:40:21  at 
> org.junit.runners.ParentRunner$2.evaluate(ParentRunner.java:293)
> 2022-09-22T04:40:21.9320401Z Sep 22 04:40:21  at 
> org.junit.runners.ParentRunner$3.evaluate(ParentRunner.java:306)
> 2022-09-22T04:40:21.9321130Z Sep 22 04:40:21  at 
> org.junit.runners.ParentRunner.run(ParentRunner.java:413)
> 2022-09-22T04:40:21.9321822Z Sep 22 04:40:21  at 
> org.junit.runner.JUnitCore.run(JUnitCore.java:137)
> 2022-09-22T04:40:21.9322498Z Sep 22 04:40:21  at 
> org.junit.runner.JUnitCore.run(JUnitCore.java:115)
> 2022-09-22T04:40:21.9323248Z Sep 22 04:40:21  at 
> org.junit.vintage.engine.execution.RunnerExecutor.execute(RunnerExecutor.java:42)
> 2022-09-22T04:40:21.9324080Z Sep 22 04:40:21  at 
> org.junit.vintage.engine.V

[jira] [Commented] (FLINK-29478) Flink sql Connector hive to support 3.1.3

2022-09-30 Thread Samrat Deb (Jira)


[ 
https://issues.apache.org/jira/browse/FLINK-29478?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17611537#comment-17611537
 ] 

Samrat Deb commented on FLINK-29478:


haha :D Thank you 

> Flink sql Connector hive to support 3.1.3 
> --
>
> Key: FLINK-29478
> URL: https://issues.apache.org/jira/browse/FLINK-29478
> Project: Flink
>  Issue Type: Improvement
>  Components: Connectors / Hive
>Reporter: Samrat Deb
>Assignee: Samrat Deb
>Priority: Minor
>
> Currently , flink-connector hive support flink-sql-connector-hive-3.1.2 as 
> highest version ! 
> h3. hive 3.1.3 released on 08 April 2022
> Proposal :- 
> We should think of adding support for 3.1.3. 



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Assigned] (FLINK-29478) Flink sql Connector hive to support 3.1.3

2022-09-30 Thread Martijn Visser (Jira)


 [ 
https://issues.apache.org/jira/browse/FLINK-29478?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Martijn Visser reassigned FLINK-29478:
--

Assignee: Samrat Deb

> Flink sql Connector hive to support 3.1.3 
> --
>
> Key: FLINK-29478
> URL: https://issues.apache.org/jira/browse/FLINK-29478
> Project: Flink
>  Issue Type: Improvement
>  Components: Connectors / Hive
>Reporter: Samrat Deb
>Assignee: Samrat Deb
>Priority: Minor
>
> Currently , flink-connector hive support flink-sql-connector-hive-3.1.2 as 
> highest version ! 
> h3. hive 3.1.3 released on 08 April 2022
> Proposal :- 
> We should think of adding support for 3.1.3. 



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Commented] (FLINK-29478) Flink sql Connector hive to support 3.1.3

2022-09-30 Thread Martijn Visser (Jira)


[ 
https://issues.apache.org/jira/browse/FLINK-29478?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17611525#comment-17611525
 ] 

Martijn Visser commented on FLINK-29478:


Ah lol that's even my own ticket :D 

I'll assign the ticket to you

> Flink sql Connector hive to support 3.1.3 
> --
>
> Key: FLINK-29478
> URL: https://issues.apache.org/jira/browse/FLINK-29478
> Project: Flink
>  Issue Type: Improvement
>  Components: Connectors / Hive
>Reporter: Samrat Deb
>Priority: Minor
>
> Currently , flink-connector hive support flink-sql-connector-hive-3.1.2 as 
> highest version ! 
> h3. hive 3.1.3 released on 08 April 2022
> Proposal :- 
> We should think of adding support for 3.1.3. 



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Comment Edited] (FLINK-29478) Flink sql Connector hive to support 3.1.3

2022-09-30 Thread Samrat Deb (Jira)


[ 
https://issues.apache.org/jira/browse/FLINK-29478?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17611522#comment-17611522
 ] 

Samrat Deb edited comment on FLINK-29478 at 9/30/22 11:36 AM:
--

Thanks [~martijnvisser] for your input 

there is an associated ticket
https://issues.apache.org/jira/browse/FLINK-27044 
dedicated to remove support for 
 * 1.*
 * 2.1.*
 * 2.2.*

all this support was removed as part of `FLINK-27044`
flink will support only latest version of hive  2.3.x and 3.1.x





As you have suggested to upgrade the hive version to the latest one 3.1.3 will 
be the way forward. 

I can now allign on the same 

can i work on this ticket to make the upgrade for 3.1.3 ?


was (Author: samrat007):
Thanks [~martijnvisser] for your input 

there is an associated ticket
https://issues.apache.org/jira/browse/FLINK-27044 
dedicated to remove support for 
 * 1.*
 * 2.1.*
 * 2.2.*
all this support was removed as part of `FLINK-27044`
flink will support only latest version of hive  2.3.x and 3.1.x
As you have suggested to upgrade the hive version to the latest one 3.1.3 will 
be the way forward 

can i work on this to make the upgrade ?

> Flink sql Connector hive to support 3.1.3 
> --
>
> Key: FLINK-29478
> URL: https://issues.apache.org/jira/browse/FLINK-29478
> Project: Flink
>  Issue Type: Improvement
>  Components: Connectors / Hive
>Reporter: Samrat Deb
>Priority: Minor
>
> Currently , flink-connector hive support flink-sql-connector-hive-3.1.2 as 
> highest version ! 
> h3. hive 3.1.3 released on 08 April 2022
> Proposal :- 
> We should think of adding support for 3.1.3. 



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Commented] (FLINK-29478) Flink sql Connector hive to support 3.1.3

2022-09-30 Thread Samrat Deb (Jira)


[ 
https://issues.apache.org/jira/browse/FLINK-29478?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17611522#comment-17611522
 ] 

Samrat Deb commented on FLINK-29478:


Thanks [~martijnvisser] for your input 

there is an associated ticket
https://issues.apache.org/jira/browse/FLINK-27044 
dedicated to remove support for 
 * 1.*
 * 2.1.*
 * 2.2.*
all this support was removed as part of `FLINK-27044`
flink will support only latest version of hive  2.3.x and 3.1.x
As you have suggested to upgrade the hive version to the latest one 3.1.3 will 
be the way forward 

can i work on this to make the upgrade ?

> Flink sql Connector hive to support 3.1.3 
> --
>
> Key: FLINK-29478
> URL: https://issues.apache.org/jira/browse/FLINK-29478
> Project: Flink
>  Issue Type: Improvement
>  Components: Connectors / Hive
>Reporter: Samrat Deb
>Priority: Minor
>
> Currently , flink-connector hive support flink-sql-connector-hive-3.1.2 as 
> highest version ! 
> h3. hive 3.1.3 released on 08 April 2022
> Proposal :- 
> We should think of adding support for 3.1.3. 



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Updated] (FLINK-28291) Add kerberos delegation token renewer feature instead of logged from keytab individually

2022-09-30 Thread Chesnay Schepler (Jira)


 [ 
https://issues.apache.org/jira/browse/FLINK-28291?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Chesnay Schepler updated FLINK-28291:
-
Fix Version/s: (was: 1.13.5)

> Add kerberos delegation token renewer feature instead of logged from keytab 
> individually
> 
>
> Key: FLINK-28291
> URL: https://issues.apache.org/jira/browse/FLINK-28291
> Project: Flink
>  Issue Type: New Feature
>  Components: Deployment / YARN
>Affects Versions: 1.13.5
>Reporter: jiulong.zhu
>Priority: Minor
>  Labels: PatchAvailable, patch-available
> Attachments: FLINK-28291.0001.patch
>
>
> h2. 1. Design
> LifeCycle of delegation token in RM:
>  # Container starts with DT given by client.
>  # Enable delegation token renewer by:
>  ## set {{security.kerberos.token.renew.enabled}} true, default false. And
>  ## specify {{security.kerberos.login.keytab}} and 
> {{security.kerberos.login.principal}}
>  # When enabled delegation token renewer, the renewer thread will re-obtain 
> tokens from DelegationTokenProvider(only HadoopFSDelegationTokenProvider 
> now). Then the renewer thread will broadcast new tokens to RM locally, all 
> JMs and all TMs by RPCGateway.
>  # RM process adds new tokens in context by UserGroupInformation.
> LifeCycle of delegation token in JM / TM:
>  # TaskManager starts with keytab stored in remote hdfs.
>  # When registered successfully, JM / TM get the current tokens of RM boxed 
> by {{JobMasterRegistrationSuccess}} / {{{}TaskExecutorRegistrationSuccess{}}}.
>  # JM / TM process add new tokens in context by UserGroupInformation.
> It’s too heavy and unnecessary to retrieval leader of ResourceManager by 
> HAService, so DelegationTokenManager is instanced by ResourceManager. So 
> DelegationToken can hold the reference of ResourceManager, instead of RM 
> RPCGateway or self gateway.
> h2. 2. Test
>  # No local junit test. It’s too heavy to build junit environments including 
> KDC and local hadoop.
>  # Cluster test
> step 1: Specify krb5.conf with short token lifetime(ticket_lifetime, 
> renew_lifetime) when submitting flink application.
> ```
> {{flink run  -yD security.kerberos.token.renew.enabled=true -yD 
> security.kerberos.krb5-conf.path= /home/work/krb5.conf -yD 
> security.kerberos.login.use-ticket-cache=false ...}}
> ```
> step 2: Watch token identifier changelog and synchronizer between rm and 
> worker.
> >> 
> In RM / JM log, 
> 2022-06-28 15:13:03,509 INFO org.apache.flink.runtime.util.HadoopUtils [] - 
> New token (HDFS_DELEGATION_TOKEN token 52101 for work on ha-hdfs:newfyyy) 
> created in KerberosDelegationToken, and next schedule delay is 64799880 ms. 
> 2022-06-28 15:13:03,529 INFO org.apache.flink.runtime.util.HadoopUtils [] - 
> Updating delegation tokens for current user. 2022-06-28 15:13:04,729 INFO 
> org.apache.flink.runtime.util.HadoopUtils [] - JobMaster receives new token 
> (HDFS_DELEGATION_TOKEN token 52101 for work on ha-hdfs:newfyyy) from RM.
> … 
> 2022-06-29 09:13:03,732 INFO org.apache.flink.runtime.util.HadoopUtils [] - 
> New token (HDFS_DELEGATION_TOKEN token 52310 for work on ha-hdfs:newfyyy) 
> created in KerberosDelegationToken, and next schedule delay is 64800045 ms.
> 2022-06-29 09:13:03,805 INFO org.apache.flink.runtime.util.HadoopUtils [] - 
> Updating delegation tokens for current user. 
> 2022-06-29 09:13:03,806 INFO org.apache.flink.runtime.util.HadoopUtils [] - 
> JobMaster receives new token (HDFS_DELEGATION_TOKEN token 52310 for work on 
> ha-hdfs:newfyyy) from RM.
> >> 
> In TM log, 
> 2022-06-28 15:13:17,983 INFO org.apache.flink.runtime.util.HadoopUtils [] - 
> TaskManager receives new token (HDFS_DELEGATION_TOKEN token 52101 for work on 
> ha-hdfs:newfyyy) from RM. 
> 2022-06-28 15:13:18,016 INFO org.apache.flink.runtime.util.HadoopUtils [] - 
> Updating delegation tokens for current user. 
> … 
> 2022-06-29 09:13:03,809 INFO org.apache.flink.runtime.util.HadoopUtils [] - 
> TaskManager receives new token (HDFS_DELEGATION_TOKEN token 52310 for work on 
> ha-hdfs:newfyyy) from RM.
> 2022-06-29 09:13:03,836 INFO org.apache.flink.runtime.util.HadoopUtils [] - 
> Updating delegation tokens for current user.



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Updated] (FLINK-29487) RpcService should implement AutoCloseableAsync

2022-09-30 Thread ASF GitHub Bot (Jira)


 [ 
https://issues.apache.org/jira/browse/FLINK-29487?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

ASF GitHub Bot updated FLINK-29487:
---
Labels: pull-request-available  (was: )

> RpcService should implement AutoCloseableAsync
> --
>
> Key: FLINK-29487
> URL: https://issues.apache.org/jira/browse/FLINK-29487
> Project: Flink
>  Issue Type: Sub-task
>  Components: Runtime / RPC
>Reporter: Chesnay Schepler
>Assignee: Chesnay Schepler
>Priority: Major
>  Labels: pull-request-available
> Fix For: 1.17.0
>
>




--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Updated] (FLINK-29488) MetricRegistryImpl should implement AutoCloseableAsync

2022-09-30 Thread ASF GitHub Bot (Jira)


 [ 
https://issues.apache.org/jira/browse/FLINK-29488?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

ASF GitHub Bot updated FLINK-29488:
---
Labels: pull-request-available  (was: )

> MetricRegistryImpl should implement AutoCloseableAsync
> --
>
> Key: FLINK-29488
> URL: https://issues.apache.org/jira/browse/FLINK-29488
> Project: Flink
>  Issue Type: Sub-task
>  Components: Runtime / Metrics
>Reporter: Chesnay Schepler
>Assignee: Chesnay Schepler
>Priority: Major
>  Labels: pull-request-available
> Fix For: 1.17.0
>
>




--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[GitHub] [flink] flinkbot commented on pull request #20933: [FLINK-29488][metrics] MetricRegistryImpl implements AutoCloseableAsync

2022-09-30 Thread GitBox


flinkbot commented on PR #20933:
URL: https://github.com/apache/flink/pull/20933#issuecomment-1263450070

   
   ## CI report:
   
   * 2b68e500b93936eba18d6d9352b5735d8f9ec898 UNKNOWN
   
   
   Bot commands
 The @flinkbot bot supports the following commands:
   
- `@flinkbot run azure` re-run the last Azure build
   


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



[GitHub] [flink] flinkbot commented on pull request #20932: [FLINK-29487][rpc] RpcService extends AutoCloseableAsync

2022-09-30 Thread GitBox


flinkbot commented on PR #20932:
URL: https://github.com/apache/flink/pull/20932#issuecomment-1263449935

   
   ## CI report:
   
   * 9255ea9840abf5ebeed5afef552f2feb9b86d59d UNKNOWN
   
   
   Bot commands
 The @flinkbot bot supports the following commands:
   
- `@flinkbot run azure` re-run the last Azure build
   


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



[jira] [Commented] (FLINK-29330) Provide better logs of MiniCluster shutdown procedure

2022-09-30 Thread Chesnay Schepler (Jira)


[ 
https://issues.apache.org/jira/browse/FLINK-29330?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17611520#comment-17611520
 ] 

Chesnay Schepler commented on FLINK-29330:
--

Should LeaderRetrievalService really implement it though? It doesn't seem to be 
doing asynchronously; or are you proposing to keep that as an implementation 
detail?

> Provide better logs of MiniCluster shutdown procedure
> -
>
> Key: FLINK-29330
> URL: https://issues.apache.org/jira/browse/FLINK-29330
> Project: Flink
>  Issue Type: Technical Debt
>  Components: Tests
>Reporter: Chesnay Schepler
>Assignee: Chesnay Schepler
>Priority: Major
> Fix For: 1.17.0
>
>
> I recently ran into an issue where the shutdown of a MiniCluster timed out. 
> The logs weren't helpful at all and I had to go in and check every 
> asynchronously component for whether _that_ component was the cause.
> The main issues were that various components don't log anything at all, or 
> that when they did it wasn't clear who owned that component.
> I'd like to add a util that makes it easier for us log the start/stop of a 
> shutdown procedure,
> {code:java}
> public class ShutdownLog {
> /**
>  * Logs the beginning and end of the shutdown procedure for the given 
> component.
>  *
>  * This method accepts a {@link Supplier} instead of a {@link 
> CompletableFuture} because the
>  * latter usually required implies the shutdown to already have begun.
>  *
>  * @param log Logger of owning component
>  * @param component component that will be shut down
>  * @param shutdownTrigger component shutdown trigger
>  * @return termination future of the component
>  */
> public static  CompletableFuture logShutdown(
> Logger log, String component, Supplier> 
> shutdownTrigger) {
> log.debug("Starting shutdown of {}.", component);
> return FutureUtils.logCompletion(log, "shutdown of " + component, 
> shutdownTrigger.get());
> }
> }
> public class FutureUtils {
> public static  CompletableFuture logCompletion(
> Logger log, String action, CompletableFuture future) {
> future.handle(
> (t, throwable) -> {
> if (throwable == null) {
> log.debug("Completed {}.", action);
> } else {
> log.debug("Failed {}.", action, throwable);
> }
> return null;
> });
> return future;
> }
> ...
> {code}
> and extend the AutoCloseableAsync interface for an easy opt-in and customized 
> logging:
> {code:java}
> default CompletableFuture closeAsync(Logger log) {
> return ShutdownLog.logShutdown(log, getClass().getSimpleName(), 
> this::closeAsync);
> }
> {code}
> MiniCluster example usages:
> {code:java}
> -terminationFutures.add(dispatcherResourceManagerComponent.closeAsync())
> +terminationFutures.add(dispatcherResourceManagerComponent.closeAsync(LOG))
> {code}
> {code:java}
> -return ExecutorUtils.nonBlockingShutdown(
> -executorShutdownTimeoutMillis, TimeUnit.MILLISECONDS, ioExecutor);
> +return ShutdownLog.logShutdown(
> +LOG,
> +"ioExecutor",
> +() ->
> +ExecutorUtils.nonBlockingShutdown(
> +executorShutdownTimeoutMillis,
> +TimeUnit.MILLISECONDS,
> +ioExecutor));
> {code}
> [~mapohl] I'm interested what you think about this.



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Comment Edited] (FLINK-29330) Provide better logs of MiniCluster shutdown procedure

2022-09-30 Thread Chesnay Schepler (Jira)


[ 
https://issues.apache.org/jira/browse/FLINK-29330?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17611520#comment-17611520
 ] 

Chesnay Schepler edited comment on FLINK-29330 at 9/30/22 11:20 AM:


Should LeaderRetrievalService really implement it though? It doesn't seem to be 
doing asynchronously; or are you proposing to change that or keep that as an 
implementation detail?


was (Author: zentol):
Should LeaderRetrievalService really implement it though? It doesn't seem to be 
doing asynchronously; or are you proposing to keep that as an implementation 
detail?

> Provide better logs of MiniCluster shutdown procedure
> -
>
> Key: FLINK-29330
> URL: https://issues.apache.org/jira/browse/FLINK-29330
> Project: Flink
>  Issue Type: Technical Debt
>  Components: Tests
>Reporter: Chesnay Schepler
>Assignee: Chesnay Schepler
>Priority: Major
> Fix For: 1.17.0
>
>
> I recently ran into an issue where the shutdown of a MiniCluster timed out. 
> The logs weren't helpful at all and I had to go in and check every 
> asynchronously component for whether _that_ component was the cause.
> The main issues were that various components don't log anything at all, or 
> that when they did it wasn't clear who owned that component.
> I'd like to add a util that makes it easier for us log the start/stop of a 
> shutdown procedure,
> {code:java}
> public class ShutdownLog {
> /**
>  * Logs the beginning and end of the shutdown procedure for the given 
> component.
>  *
>  * This method accepts a {@link Supplier} instead of a {@link 
> CompletableFuture} because the
>  * latter usually required implies the shutdown to already have begun.
>  *
>  * @param log Logger of owning component
>  * @param component component that will be shut down
>  * @param shutdownTrigger component shutdown trigger
>  * @return termination future of the component
>  */
> public static  CompletableFuture logShutdown(
> Logger log, String component, Supplier> 
> shutdownTrigger) {
> log.debug("Starting shutdown of {}.", component);
> return FutureUtils.logCompletion(log, "shutdown of " + component, 
> shutdownTrigger.get());
> }
> }
> public class FutureUtils {
> public static  CompletableFuture logCompletion(
> Logger log, String action, CompletableFuture future) {
> future.handle(
> (t, throwable) -> {
> if (throwable == null) {
> log.debug("Completed {}.", action);
> } else {
> log.debug("Failed {}.", action, throwable);
> }
> return null;
> });
> return future;
> }
> ...
> {code}
> and extend the AutoCloseableAsync interface for an easy opt-in and customized 
> logging:
> {code:java}
> default CompletableFuture closeAsync(Logger log) {
> return ShutdownLog.logShutdown(log, getClass().getSimpleName(), 
> this::closeAsync);
> }
> {code}
> MiniCluster example usages:
> {code:java}
> -terminationFutures.add(dispatcherResourceManagerComponent.closeAsync())
> +terminationFutures.add(dispatcherResourceManagerComponent.closeAsync(LOG))
> {code}
> {code:java}
> -return ExecutorUtils.nonBlockingShutdown(
> -executorShutdownTimeoutMillis, TimeUnit.MILLISECONDS, ioExecutor);
> +return ShutdownLog.logShutdown(
> +LOG,
> +"ioExecutor",
> +() ->
> +ExecutorUtils.nonBlockingShutdown(
> +executorShutdownTimeoutMillis,
> +TimeUnit.MILLISECONDS,
> +ioExecutor));
> {code}
> [~mapohl] I'm interested what you think about this.



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Created] (FLINK-29488) MetricRegistryImpl should implement AutoCloseableAsync

2022-09-30 Thread Chesnay Schepler (Jira)
Chesnay Schepler created FLINK-29488:


 Summary: MetricRegistryImpl should implement AutoCloseableAsync
 Key: FLINK-29488
 URL: https://issues.apache.org/jira/browse/FLINK-29488
 Project: Flink
  Issue Type: Sub-task
  Components: Runtime / Metrics
Reporter: Chesnay Schepler
Assignee: Chesnay Schepler
 Fix For: 1.17.0






--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Created] (FLINK-29487) RpcService should implement AutoCloseableAsync

2022-09-30 Thread Chesnay Schepler (Jira)
Chesnay Schepler created FLINK-29487:


 Summary: RpcService should implement AutoCloseableAsync
 Key: FLINK-29487
 URL: https://issues.apache.org/jira/browse/FLINK-29487
 Project: Flink
  Issue Type: Sub-task
  Components: Runtime / Coordination
Reporter: Chesnay Schepler
Assignee: Chesnay Schepler
 Fix For: 1.17.0






--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Updated] (FLINK-29487) RpcService should implement AutoCloseableAsync

2022-09-30 Thread Chesnay Schepler (Jira)


 [ 
https://issues.apache.org/jira/browse/FLINK-29487?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Chesnay Schepler updated FLINK-29487:
-
Component/s: Runtime / RPC
 (was: Runtime / Coordination)

> RpcService should implement AutoCloseableAsync
> --
>
> Key: FLINK-29487
> URL: https://issues.apache.org/jira/browse/FLINK-29487
> Project: Flink
>  Issue Type: Sub-task
>  Components: Runtime / RPC
>Reporter: Chesnay Schepler
>Assignee: Chesnay Schepler
>Priority: Major
> Fix For: 1.17.0
>
>




--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Updated] (FLINK-16908) FlinkKafkaProducerITCase testScaleUpAfterScalingDown Timeout expired while initializing transactional state in 60000ms.

2022-09-30 Thread Piotr Nowojski (Jira)


 [ 
https://issues.apache.org/jira/browse/FLINK-16908?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Piotr Nowojski updated FLINK-16908:
---
Fix Version/s: (was: 1.17.0)

> FlinkKafkaProducerITCase testScaleUpAfterScalingDown Timeout expired while 
> initializing transactional state in 6ms.
> ---
>
> Key: FLINK-16908
> URL: https://issues.apache.org/jira/browse/FLINK-16908
> Project: Flink
>  Issue Type: Bug
>  Components: Connectors / Kafka
>Affects Versions: 1.11.0, 1.12.0, 1.13.2, 1.14.3
>Reporter: Piotr Nowojski
>Assignee: Fabian Paul
>Priority: Minor
>  Labels: auto-deprioritized-major, stale-assigned, test-stability
>
> https://dev.azure.com/rmetzger/Flink/_build/results?buildId=6889&view=logs&j=c5f0071e-1851-543e-9a45-9ac140befc32&t=f66652e3-384e-5b25-be29-abfea69ea8da
> {noformat}
> [ERROR] 
> testScaleUpAfterScalingDown(org.apache.flink.streaming.connectors.kafka.FlinkKafkaProducerITCase)
>   Time elapsed: 64.353 s  <<< ERROR!
> org.apache.kafka.common.errors.TimeoutException: Timeout expired while 
> initializing transactional state in 6ms.
> {noformat}
> After this initial error many other tests (I think all following unit tests) 
> failed with errors like:
> {noformat}
> [ERROR] 
> testFailAndRecoverSameCheckpointTwice(org.apache.flink.streaming.connectors.kafka.FlinkKafkaProducerITCase)
>   Time elapsed: 7.895 s  <<< FAILURE!
> java.lang.AssertionError: Detected producer leak. Thread name: 
> kafka-producer-network-thread | producer-196
>   at 
> org.apache.flink.streaming.connectors.kafka.FlinkKafkaProducerITCase.checkProducerLeak(FlinkKafkaProducerITCase.java:675)
>   at 
> org.apache.flink.streaming.connectors.kafka.FlinkKafkaProducerITCase.testFailAndRecoverSameCheckpointTwice(FlinkKafkaProducerITCase.java:311)
> {noformat}



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Updated] (FLINK-21307) Revisit activation model of FlinkSecurityManager

2022-09-30 Thread Piotr Nowojski (Jira)


 [ 
https://issues.apache.org/jira/browse/FLINK-21307?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Piotr Nowojski updated FLINK-21307:
---
Fix Version/s: (was: 1.17.0)

> Revisit activation model of FlinkSecurityManager
> 
>
> Key: FLINK-21307
> URL: https://issues.apache.org/jira/browse/FLINK-21307
> Project: Flink
>  Issue Type: Bug
>  Components: Runtime / Task
>Affects Versions: 1.13.0
>Reporter: Robert Metzger
>Priority: Minor
>  Labels: auto-deprioritized-critical, auto-deprioritized-major
>
> In FLINK-15156, we introduced a feature that allows users to log or 
> completely disable calls to System.exit(). This feature is enabled for 
> certain threads / code sections intended to execute user-code.
> The activation of the security manager (for monitoring user calls to 
> System.exit() is currently not well-defined, and only implemented on a 
> best-effort basis.
> This ticket is to revisit the activation.



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Updated] (FLINK-15550) testCancelTaskExceptionAfterTaskMarkedFailed failed on azure

2022-09-30 Thread Piotr Nowojski (Jira)


 [ 
https://issues.apache.org/jira/browse/FLINK-15550?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Piotr Nowojski updated FLINK-15550:
---
Fix Version/s: (was: 1.17.0)

> testCancelTaskExceptionAfterTaskMarkedFailed failed on azure
> 
>
> Key: FLINK-15550
> URL: https://issues.apache.org/jira/browse/FLINK-15550
> Project: Flink
>  Issue Type: Bug
>  Components: Runtime / Task
>Affects Versions: 1.11.0, 1.12.5, 1.13.6, 1.14.3
>Reporter: Yun Tang
>Priority: Major
>  Labels: pull-request-available, stale-assigned
>
> Instance: 
> https://dev.azure.com/rmetzger/Flink/_build/results?buildId=4241&view=ms.vss-test-web.build-test-results-tab&runId=12434&resultId=108939&paneView=debug
> {code:java}
> java.lang.AssertionError: expected: but was:
>   at 
> org.apache.flink.runtime.taskmanager.TaskTest.testCancelTaskExceptionAfterTaskMarkedFailed(TaskTest.java:525)
> {code}
> {code:java}
> expected: but was:
> {code}



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Updated] (FLINK-26568) BlockingShuffleITCase.testDeletePartitionFileOfBoundedBlockingShuffle timing out on Azure

2022-09-30 Thread Piotr Nowojski (Jira)


 [ 
https://issues.apache.org/jira/browse/FLINK-26568?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Piotr Nowojski updated FLINK-26568:
---
Fix Version/s: (was: 1.17.0)

> BlockingShuffleITCase.testDeletePartitionFileOfBoundedBlockingShuffle timing 
> out on Azure
> -
>
> Key: FLINK-26568
> URL: https://issues.apache.org/jira/browse/FLINK-26568
> Project: Flink
>  Issue Type: Bug
>  Components: Runtime / Network, Runtime / Task
>Affects Versions: 1.15.0
>Reporter: Matthias Pohl
>Priority: Major
>  Labels: test-stability
> Attachments: image-2022-03-22-18-19-53-171.png
>
>
> [This 
> build|https://dev.azure.com/mapohl/flink/_build/results?buildId=845&view=logs&j=0a15d512-44ac-5ba5-97ab-13a5d066c22c&t=9a028d19-6c4b-5a4e-d378-03fca149d0b1&l=12865]
>  timed out due the test 
> {{BlockingShuffleITCase.testDeletePartitionFileOfBoundedBlockingShuffle}} not 
> finishing.



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Updated] (FLINK-23970) Split off the behaviour for finished StreamTask(s)

2022-09-30 Thread Piotr Nowojski (Jira)


 [ 
https://issues.apache.org/jira/browse/FLINK-23970?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Piotr Nowojski updated FLINK-23970:
---
Fix Version/s: (was: 1.17.0)

> Split off the behaviour for finished StreamTask(s)
> --
>
> Key: FLINK-23970
> URL: https://issues.apache.org/jira/browse/FLINK-23970
> Project: Flink
>  Issue Type: Sub-task
>  Components: Runtime / Checkpointing
>Reporter: Yun Gao
>Priority: Major
>
> We will have a double check on how we could better abstract the behavior of 
> the tasks marked as finished on recovery, the target is to make the behaviors 
> implemented as centralized as we can (like introducing specialized 
> _FinishedStreamTask_).



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[GitHub] [flink] snuyanzin commented on pull request #20922: [DRAFT] Apply changes for calcite 1.27

2022-09-30 Thread GitBox


snuyanzin commented on PR #20922:
URL: https://github.com/apache/flink/pull/20922#issuecomment-1263431047

   @flinkbot run azure


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



[GitHub] [flink] ferenc-csaky commented on a diff in pull request #20542: [FLINK-28910][Connectors/hbase]Fix potential data deletion while updating HBase rows

2022-09-30 Thread GitBox


ferenc-csaky commented on code in PR #20542:
URL: https://github.com/apache/flink/pull/20542#discussion_r984470999


##
flink-connectors/flink-connector-hbase-base/src/main/java/org/apache/flink/connector/hbase/sink/HBaseSinkFunction.java:
##
@@ -191,8 +195,10 @@ private void checkErrorAndRethrow() {
 public void invoke(T value, Context context) throws Exception {
 checkErrorAndRethrow();
 
-mutator.mutate(mutationConverter.convertToMutation(value));
-
+Mutation mutation = mutationConverter.convertToMutation(value);
+synchronized (mutationMap) {
+mutationMap.put(mutation.getRow(), mutation);

Review Comment:
   To overcome this, I suggest to convert it to a base64 string, e.g.:
   ```java
   String key = Base64.getEncoder().encodeToString(mutation.getRow());
   ```
   
   Or to create a simple wrapper class, where the `equals` and `hashCode` are 
overridden properly for arrays.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



[jira] [Commented] (FLINK-28910) CDC From Mysql To Hbase Bugs

2022-09-30 Thread Ferenc Csaky (Jira)


[ 
https://issues.apache.org/jira/browse/FLINK-28910?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17611505#comment-17611505
 ] 

Ferenc Csaky commented on FLINK-28910:
--

My HBase knowledge is a bit rusty, so bear with me if any of my assumptions is 
not correct. As per my understanding, if we would like to move towards the 
atomic table operations way, we would lose the currently leveraged buffering 
functionality, because it is possible to define multiple mutations for 1 
specific row with {{{}RowMutations{}}}, but that's it, the operations will be 
sent right away, which will probably affect performance.

According to the PR, it will do the job I think, would that be possible to 
handle the action itself smarter? I'm wondering about would it make sense to 
omit the delete op. in specific cases?

> CDC From Mysql To Hbase Bugs
> 
>
> Key: FLINK-28910
> URL: https://issues.apache.org/jira/browse/FLINK-28910
> Project: Flink
>  Issue Type: Bug
>  Components: Connectors / HBase
>Reporter: TE
>Priority: Major
>  Labels: pull-request-available, stale-blocker
>
> I use Flink for CDC from Mysql to Hbase.
> The problem I encountered is that the Mysql record is updated (not deleted), 
> but the record in hbase is deleted sometimes.
> I tried to analyze the problem and found the reason as follows:
> The update action of Mysql will be decomposed into delete + insert by Flink.
> The Hbase connector uses a mutator to handle this set of actions.
> However, if the order of these actions is not actively set, the processing of 
> the mutator will not guarantee the order of execution.
> Therefore, when the update of Mysql is triggered, it is possible that hbase 
> actually performed the actions in the order of put + delete, resulting in the 
> data being deleted.



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Commented] (FLINK-29478) Flink sql Connector hive to support 3.1.3

2022-09-30 Thread Martijn Visser (Jira)


[ 
https://issues.apache.org/jira/browse/FLINK-29478?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17611497#comment-17611497
 ] 

Martijn Visser commented on FLINK-29478:


Instead of keep adding newer versions of Hive to support, we should actually 
bring that number down imho. Hive 1 is meant for Hadoop 1, which isn't 
supported by the Hadoop community anymore. So why not drop that version? Why 
not only support Hive 2.3.9 and 3.1.3, perhaps also Hive 2.2.0. A patch version 
shouldn't lead to a breaking change, then it should have been a minor or major 
version upgrade. 

> Flink sql Connector hive to support 3.1.3 
> --
>
> Key: FLINK-29478
> URL: https://issues.apache.org/jira/browse/FLINK-29478
> Project: Flink
>  Issue Type: Improvement
>  Components: Connectors / Hive
>Reporter: Samrat Deb
>Priority: Minor
>
> Currently , flink-connector hive support flink-sql-connector-hive-3.1.2 as 
> highest version ! 
> h3. hive 3.1.3 released on 08 April 2022
> Proposal :- 
> We should think of adding support for 3.1.3. 



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[GitHub] [flink] zentol commented on a diff in pull request #20893: [FLINK-29403][cep] Streamline SimpleCondition usage

2022-09-30 Thread GitBox


zentol commented on code in PR #20893:
URL: https://github.com/apache/flink/pull/20893#discussion_r984453514


##
flink-libraries/flink-cep/src/main/java/org/apache/flink/cep/pattern/conditions/SimpleCondition.java:
##
@@ -40,4 +40,13 @@
 public boolean filter(T value, Context ctx) throws Exception {
 return filter(value);
 }
+
+public static  SimpleCondition of(FilterFunction filters) {

Review Comment:
   At the end of the day this doesn't need to be the final solution.
   
   While `Pattern#where(FilterFunction)` is a lot more user-friendly this would 
be a breaking change, because an existing `SimpleCondition` would satisfy both 
signatures which the compiler rejects.
   I'd like to avoid using different method names for the `FilterFunction` 
variants.
   
   We could also modify `SimpleCondition` to not implement the `FilterFunction` 
interface, but this would again be a potentially breaking change.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



[jira] [Commented] (FLINK-29437) The partition of data before and after the Kafka Shuffle are not aligned

2022-09-30 Thread Zakelly Lan (Jira)


[ 
https://issues.apache.org/jira/browse/FLINK-29437?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17611494#comment-17611494
 ] 

Zakelly Lan commented on FLINK-29437:
-

After offline sync with [~ym] , we agree to fix this as I proposed.

> The partition of data before and after the Kafka Shuffle are not aligned
> 
>
> Key: FLINK-29437
> URL: https://issues.apache.org/jira/browse/FLINK-29437
> Project: Flink
>  Issue Type: Bug
>  Components: API / DataStream
>Affects Versions: 1.15.2
>Reporter: Zakelly Lan
>Assignee: Zakelly Lan
>Priority: Major
> Attachments: image-2022-09-28-14-32-28-116.png, 
> image-2022-09-28-14-35-47-954.png
>
>
> I notice that the key group range in consumer side of Kafka Shuffle is not 
> aligned with the producer side, there are two problems:
>  # The data partitioning of the sink(producer) is exactly the same way as a 
> keyed stream that as the same maximum parallelism as the number of kafka 
> partitions does, but in consumer side the number of partitions and key groups 
> are not the same.
>  # There is a distribution of assigning kafka partitions to consumer subtasks 
> (See KafkaTopicPartitionAssigner#assign), but the producer of Kafka Shuffle 
> simply assume the partition index equals the subtask index. e.g.
>        !image-2022-09-28-14-32-28-116.png|width=1133,height=274!
> My proposed change:
>  # Set the max parallelism of the key stream in consumer side as the number 
> of kafka partitions. 
>  # Use the same method when assigning kafka partitions to consumer subtasks 
> to maintain a map from subtasks to kafka partitions, which is used by the 
> producer to insert into the right partition for data for a subtask. i.e.
>        !image-2022-09-28-14-35-47-954.png|width=1030,height=283!



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Updated] (FLINK-29480) Skip invalid messages when writing

2022-09-30 Thread Salva (Jira)


 [ 
https://issues.apache.org/jira/browse/FLINK-29480?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Salva updated FLINK-29480:
--
Description: 
As reported in [1], it seems that it's not possible to skip invalid messages 
when writing. More specifically, if there is an error serializing messages, 
there is no option for skipping them and then Flink job enters a crash loop. In 
particular, the `write` method of the `KafkaWriter` looks like this:
{code:java}
@Override
public void write(IN element, Context context) throws IOException {
  final ProducerRecord record = 
recordSerializer.serialize(element, ...);
  currentProducer.send(record, deliveryCallback); // line 200
  numRecordsSendCounter.inc();
} {code}
So, If you make your `serialize` method return `null`, this is what you get at 
runtime
{code:java}
java.lang.NullPointerException at 
org.apache.kafka.clients.producer.KafkaProducer.doSend(KafkaProducer.java:906) 
at org.apache.kafka.clients.producer.KafkaProducer.send(KafkaProducer.java:885) 
at 
org.apache.flink.connector.kafka.sink.KafkaWriter.write(KafkaWriter.java:200) 
at 
org.apache.flink.streaming.runtime.operators.sink.SinkWriterOperator.processElement(SinkWriterOperator.java:158)
  {code}
What I propose is to modify the KafkaWriter like this:
{code:java}
@Override
public void write(IN element, Context context) throws IOException {
  final ProducerRecord record = 
recordSerializer.serialize(element, ...);
  if (record != null) { // skip null records
currentProducer.send(record, deliveryCallback);
numRecordsSendCounter.inc();
  }
} {code}
In order to at least give a chance of skipping those messages and move on to 
the next ones.

Obviously, one could prepend the sink with a flatMap operator for filtering out 
invalid messages, but
 # It looks weird that one has to prepend an operator for "making sure" that 
the serializer will not fail right after. Wouldn't it be simpler to skip the 
null records directly in order to avoid this pre-check? [6]
 # It's such a simple change (apparently)
 # Brings consistency/symmetry with the reading case [5, 6]

To expand on point 3, by looking at `KafkaDeserializationSchema`:
{code:java}
@Override
T deserialize(ConsumerRecord record) throws Exception; default 
void deserialize(ConsumerRecord message, Collector out) 
throws Exception {
  T deserialized = deserialize(message);
  if (deserialized != null) { <-- skip invalid messages
out.collect(deserialized);
  }
}  {code}
one can simply return `null` in the overriden `deserialize` method in order to 
skip any message that fails to be deserialized. Similarly, if one uses the 
`KafkaRecordDeserializationSchema` interface instead:
{code:java}
void deserialize(ConsumerRecord record, Collector out) 
throws IOException {code}
then it's also possible not to invoke `out.collect(...)` on null records. To 
me, it looks strange that the same flexibility is not given on the writing case.

*References*

[1] [https://lists.apache.org/thread/ykmy4llovrrrzlvz0ng3x5yosskjg70h]

[2] 
[https://nightlies.apache.org/flink/flink-docs-release-1.14/release-notes/flink-1.14/#port-kafkasink-to-new-unified-sink-api-flip-143]
 

[3] 
[https://github.com/apache/flink/blob/f0fe85a50920da2b7d7da815db0a924940522e28/flink-connectors/flink-connector-kafka/src/main/java/org/apache/flink/connector/kafka/sink/KafkaWriter.java#L197]
 

[5] 
[https://stackoverflow.com/questions/55538736/how-to-skip-corrupted-messages-in-flink]
 

[6] [https://lists.apache.org/thread/pllv5dqq27xkvj6p3lj91vcz409pw38d] 

 

  was:
As reported in [1], it seems that it's not possible to skip invalid messages 
when writing. More specifically, if there is an error serializing messages, 
there is no option for skipping them and then Flink job enters a crash loop. In 
particular, the `write` method of the `KafkaWriter` looks like this:
{code:java}
@Override
public void write(IN element, Context context) throws IOException {
  final ProducerRecord record = 
recordSerializer.serialize(element, ...);
  currentProducer.send(record, deliveryCallback); // line 200
  numRecordsSendCounter.inc();
} {code}
So, If you make your `serialize` method return `null`, this is what you get at 
runtime
{code:java}
java.lang.NullPointerException at 
org.apache.kafka.clients.producer.KafkaProducer.doSend(KafkaProducer.java:906) 
at org.apache.kafka.clients.producer.KafkaProducer.send(KafkaProducer.java:885) 
at 
org.apache.flink.connector.kafka.sink.KafkaWriter.write(KafkaWriter.java:200) 
at 
org.apache.flink.streaming.runtime.operators.sink.SinkWriterOperator.processElement(SinkWriterOperator.java:158)
  {code}
What I propose is to modify the KafkaWriter like this:
{code:java}
@Override
public void write(IN element, Context context) throws IOException {
  final ProducerRecord record = 
recordSerializer.serialize(element, ...);
  if (record != null) { // skip null records
currentProducer.send(record, deliveryCallback);
n

[jira] [Updated] (FLINK-29480) Skip invalid messages when writing

2022-09-30 Thread Salva (Jira)


 [ 
https://issues.apache.org/jira/browse/FLINK-29480?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Salva updated FLINK-29480:
--
Description: 
As reported in [1], it seems that it's not possible to skip invalid messages 
when writing. More specifically, if there is an error serializing messages, 
there is no option for skipping them and then Flink job enters a crash loop. In 
particular, the `write` method of the `KafkaWriter` looks like this:
{code:java}
@Override
public void write(IN element, Context context) throws IOException {
  final ProducerRecord record = 
recordSerializer.serialize(element, ...);
  currentProducer.send(record, deliveryCallback); // line 200
  numRecordsSendCounter.inc();
} {code}
So, If you make your `serialize` method return `null`, this is what you get at 
runtime
{code:java}
java.lang.NullPointerException at 
org.apache.kafka.clients.producer.KafkaProducer.doSend(KafkaProducer.java:906) 
at org.apache.kafka.clients.producer.KafkaProducer.send(KafkaProducer.java:885) 
at 
org.apache.flink.connector.kafka.sink.KafkaWriter.write(KafkaWriter.java:200) 
at 
org.apache.flink.streaming.runtime.operators.sink.SinkWriterOperator.processElement(SinkWriterOperator.java:158)
  {code}
What I propose is to modify the KafkaWriter like this:
{code:java}
@Override
public void write(IN element, Context context) throws IOException {
  final ProducerRecord record = 
recordSerializer.serialize(element, ...);
  if (record != null) { // skip null records
currentProducer.send(record, deliveryCallback);
numRecordsSendCounter.inc();
  }
} {code}
In order to at least give a chance of skipping those messages and move on to 
the next ones.

Obviously, one could prepend the sink with a flatMap operator for filtering out 
invalid messages, but
 # It looks weird that one has to prepend an operator for "making sure" that 
the serializer will not fail right after. Wouldn't it be simpler to skip the 
null records directly in order to avoid this pre-check? [6]
 # It's such a simple change (apparently)
 # Brings consistency/symmetry with the reading case [5, 6]

To expand on point 3, by looking at `KafkaDeserializationSchema`:
{code:java}
@Override
T deserialize(ConsumerRecord record) throws Exception; default 
void deserialize(ConsumerRecord message, Collector out) 
throws Exception {
  T deserialized = deserialize(message);
  if (deserialized != null) { <-- skip invalid messages
out.collect(deserialized);
  }
}  {code}
one can simply return `null` in the overriden `deserialize` method in order to 
skip any message that fails to be deserialized. Similarly, if one uses the 
`KafkaRecordDeserializationSchema` interface instead:
{code:java}
void deserialize(ConsumerRecord record, Collector out) 
throws IOException {code}
Then it's also possible not to invoke `out.collect(...)` on null records. To 
me, it looks strange that the same flexibility is not given on the writing case.

*References*

[1] [https://lists.apache.org/thread/ykmy4llovrrrzlvz0ng3x5yosskjg70h]

[2] 
[https://nightlies.apache.org/flink/flink-docs-release-1.14/release-notes/flink-1.14/#port-kafkasink-to-new-unified-sink-api-flip-143]
 

[3] 
[https://github.com/apache/flink/blob/f0fe85a50920da2b7d7da815db0a924940522e28/flink-connectors/flink-connector-kafka/src/main/java/org/apache/flink/connector/kafka/sink/KafkaWriter.java#L197]
 

[5] 
[https://stackoverflow.com/questions/55538736/how-to-skip-corrupted-messages-in-flink]
 

[6] [https://lists.apache.org/thread/pllv5dqq27xkvj6p3lj91vcz409pw38d] 

 

  was:
As reported in [1], it seems that it's not possible to skip invalid messages 
when writing. More specifically, if there is an error serializing messages, 
there is no option for skipping them and then Flink job enters a crash loop. In 
particular, the `write` method of the `KafkaWriter` looks like this:
{code:java}
@Override
public void write(IN element, Context context) throws IOException {
  final ProducerRecord record = 
recordSerializer.serialize(element, ...);
  currentProducer.send(record, deliveryCallback); // line 200
  numRecordsSendCounter.inc();
} {code}
So, If you make your `serialize` method return `null`, this is what you get at 
runtime
{code:java}
java.lang.NullPointerException at 
org.apache.kafka.clients.producer.KafkaProducer.doSend(KafkaProducer.java:906) 
at org.apache.kafka.clients.producer.KafkaProducer.send(KafkaProducer.java:885) 
at 
org.apache.flink.connector.kafka.sink.KafkaWriter.write(KafkaWriter.java:200) 
at 
org.apache.flink.streaming.runtime.operators.sink.SinkWriterOperator.processElement(SinkWriterOperator.java:158)
  {code}
What I propose is to modify the KafkaWriter like this:
{code:java}
@Override
public void write(IN element, Context context) throws IOException {
  final ProducerRecord record = 
recordSerializer.serialize(element, ...);
  if (record != null) { // skip null records
currentProducer.send(record, deliveryCallback);
n

[GitHub] [flink] flinkbot commented on pull request #20931: [FLINK-29486][sql-client] Implement ClientParser for implementing remote SQL client later

2022-09-30 Thread GitBox


flinkbot commented on PR #20931:
URL: https://github.com/apache/flink/pull/20931#issuecomment-1263369448

   
   ## CI report:
   
   * 32763443345314018b4c308bc6a8d83c43491fdf UNKNOWN
   
   
   Bot commands
 The @flinkbot bot supports the following commands:
   
- `@flinkbot run azure` re-run the last Azure build
   


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



[GitHub] [flink] flinkbot commented on pull request #20930: [FLINK-29455] Introduce OperatorIdentifier

2022-09-30 Thread GitBox


flinkbot commented on PR #20930:
URL: https://github.com/apache/flink/pull/20930#issuecomment-1263364132

   
   ## CI report:
   
   * 3366ecff5109a8ba5c64a4b2b603f319d0bc3750 UNKNOWN
   
   
   Bot commands
 The @flinkbot bot supports the following commands:
   
- `@flinkbot run azure` re-run the last Azure build
   


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



[GitHub] [flink] yuzelin opened a new pull request, #20931: [FLINK-29486][sql-client] Implement ClientParser for implementing remote SQL client later

2022-09-30 Thread GitBox


yuzelin opened a new pull request, #20931:
URL: https://github.com/apache/flink/pull/20931

   
   
   ## What is the purpose of the change
   The current `SqlCommandParserImpl` for SQL client use the paser of embedded 
`TableEnvironment`, which cannot be fetched in remote mode. So this PR 
introduce a new parser.
   
   
   ## Brief change log
 - Adjust the package structure for parsers.
 - Modify `SqlCommandParser` for convenience: 
   - extends `FlinkSqlParserImplConstants` to use defined token kinds
   - add `parseStatement` method.
 - implement `ClientParser` and add corresponding test.
   
   ## Verifying this change
   
   This change added test: `ClientParserTest`.
   
   ## Does this pull request potentially affect one of the following parts:
   
 - Dependencies (does it add or upgrade a dependency): (**yes** / no)
 - The public API, i.e., is any changed class annotated with 
`@Public(Evolving)`: (yes / **no**)
 - The serializers: (yes / **no** / don't know)
 - The runtime per-record code paths (performance sensitive): (yes / **no** 
/ don't know)
 - Anything that affects deployment or recovery: JobManager (and its 
components), Checkpointing, Kubernetes/Yarn, ZooKeeper: (yes / **no** / don't 
know)
 - The S3 file system connector: (yes / **no** / don't know)
   
   ## Documentation
   
 - Does this pull request introduce a new feature? (**yes** / no)
 - If yes, how is the feature documented? (not applicable / docs / 
**JavaDocs** / not documented)
   


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



[jira] [Commented] (FLINK-29478) Flink sql Connector hive to support 3.1.3

2022-09-30 Thread Samrat Deb (Jira)


[ 
https://issues.apache.org/jira/browse/FLINK-29478?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17611478#comment-17611478
 ] 

Samrat Deb commented on FLINK-29478:


Thank you [~dannycranmer] for addressing to this issue . 

currently flink-sql-connector-hive has the following variation 
 * flink-sql-connector-hive-1.2.2
 * flink-sql-connector-hive-2.2.0
 * flink-sql-connector-hive-2.3.6
 * flink-sql-connector-hive-2.3.9
 * flink-sql-connector-hive-3.1.2



My intention of creating this ticket is to add support for hive new version 
3.1.3 . Bumping version 3.1.2 would / may lead to breaking change.

I think Providing support for hive 3.1.3 would need seperate 
`flink-sql-connector-hive-3.1.3` that will be dedicated to flink connector 
integration with hive for specific version 3.1.3 

Out of curiousity , I have a thought , 
can we generalize this flink-sql-connector-hive , and it uses version from 
runtime classes ? 
just a thought to improve more and skip the version specific support 

> Flink sql Connector hive to support 3.1.3 
> --
>
> Key: FLINK-29478
> URL: https://issues.apache.org/jira/browse/FLINK-29478
> Project: Flink
>  Issue Type: Improvement
>  Components: Connectors / Hive
>Reporter: Samrat Deb
>Priority: Minor
>
> Currently , flink-connector hive support flink-sql-connector-hive-3.1.2 as 
> highest version ! 
> h3. hive 3.1.3 released on 08 April 2022
> Proposal :- 
> We should think of adding support for 3.1.3. 



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Updated] (FLINK-29486) Enable SQL Client to Connect SQL Gateway in Remote Mode

2022-09-30 Thread ASF GitHub Bot (Jira)


 [ 
https://issues.apache.org/jira/browse/FLINK-29486?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

ASF GitHub Bot updated FLINK-29486:
---
Labels: pull-request-available  (was: )

> Enable SQL Client to Connect SQL Gateway in Remote Mode
> ---
>
> Key: FLINK-29486
> URL: https://issues.apache.org/jira/browse/FLINK-29486
> Project: Flink
>  Issue Type: New Feature
>  Components: Table SQL / Client
>Affects Versions: 1.17.0
>Reporter: yuzelin
>Priority: Major
>  Labels: pull-request-available
>
> In 
> [FLIP-24|https://cwiki.apache.org/confluence/display/FLINK/FLIP-24%3A+SQL+Client]
>  and  
> [FLIP-91|https://cwiki.apache.org/confluence/display/FLINK/FLIP-91%3A+Support+SQL+Gateway#FLIP91:SupportSQLGateway],
>  the SQL client was designed to has two mode: embedded and remote. Currently 
> only embedded mode has been implemented, so we need to implement the remote 
> mode.



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Updated] (FLINK-29455) Add OperatorIdentifier

2022-09-30 Thread ASF GitHub Bot (Jira)


 [ 
https://issues.apache.org/jira/browse/FLINK-29455?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

ASF GitHub Bot updated FLINK-29455:
---
Labels: pull-request-available  (was: )

> Add OperatorIdentifier
> --
>
> Key: FLINK-29455
> URL: https://issues.apache.org/jira/browse/FLINK-29455
> Project: Flink
>  Issue Type: Sub-task
>  Components: API / State Processor
>Reporter: Chesnay Schepler
>Assignee: Chesnay Schepler
>Priority: Major
>  Labels: pull-request-available
> Fix For: 1.17.0
>
>
> Add a class for identifying operators, that supports both uids and uidhashes, 
> and integrate into the low-level APIs.



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[GitHub] [flink] zentol opened a new pull request, #20930: [FLINK-29455] Introduce OperatorIdentifier

2022-09-30 Thread GitBox


zentol opened a new pull request, #20930:
URL: https://github.com/apache/flink/pull/20930

   This PR introduces an `OperatorIdentifier`, an abstraction over uid 
(hashes), and integrates it into the internal state-processing APIs.
   
   In a follow-up PR we'll adjust the user-facing APIs to allow the user to 
pass a uid hash.


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



[GitHub] [flink-ml] zhipeng93 merged pull request #155: [FLINK-29318] Add Transformer for PolynomialExpansion

2022-09-30 Thread GitBox


zhipeng93 merged PR #155:
URL: https://github.com/apache/flink-ml/pull/155


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



[GitHub] [flink-ml] zhipeng93 commented on pull request #155: [FLINK-29318] Add Transformer for PolynomialExpansion

2022-09-30 Thread GitBox


zhipeng93 commented on PR #155:
URL: https://github.com/apache/flink-ml/pull/155#issuecomment-1263355508

   Thanks for the update. Merging.


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



[jira] [Commented] (FLINK-29479) support whether using system env for pythonpath

2022-09-30 Thread Huang Xingbo (Jira)


[ 
https://issues.apache.org/jira/browse/FLINK-29479?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17611469#comment-17611469
 ] 

Huang Xingbo commented on FLINK-29479:
--

[~jackylau] From my side, the users dependency will have a higher priority, so 
it shouldn't be affected by the PYTHONPATH of the system env  under normal 
circumstances. Could you give a more specific example of how the system's 
PYTHONPATH is having a bad effect in some circumstance.

> support whether using system env for pythonpath
> ---
>
> Key: FLINK-29479
> URL: https://issues.apache.org/jira/browse/FLINK-29479
> Project: Flink
>  Issue Type: Bug
>  Components: API / Python
>Affects Versions: 1.17.0
>Reporter: jackylau
>Priority: Major
> Fix For: 1.17.0
>
>
> It exists PYTHONPATH env in system,like yarn/k8s images, it will cause 
> conflict with users python depdendency sometimes. so i suggest add a config 
> to do whether using system env of PYTHONPATH



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Updated] (FLINK-29486) Enable SQL Client to Connect SQL Gateway in Remote Mode

2022-09-30 Thread yuzelin (Jira)


 [ 
https://issues.apache.org/jira/browse/FLINK-29486?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

yuzelin updated FLINK-29486:

Description: In [FLIP-24] and  
[FLIP-91|https://cwiki.apache.org/confluence/display/FLINK/FLIP-91%3A+Support+SQL+Gateway#FLIP91:SupportSQLGateway],
 the SQL client was designed to has two mode: embedded and remote. Currently 
only embedded mode has been implemented, so we need to implement the remote 
mode.  (was: In [FLIP-24|http://example.com] and  [FLIP-91|], the SQL client 
was designed to has two mode: embedded and remote. Currently only embedded mode 
has been implemented, so we need to implement the remote mode.)

> Enable SQL Client to Connect SQL Gateway in Remote Mode
> ---
>
> Key: FLINK-29486
> URL: https://issues.apache.org/jira/browse/FLINK-29486
> Project: Flink
>  Issue Type: New Feature
>  Components: Table SQL / Client
>Affects Versions: 1.17.0
>Reporter: yuzelin
>Priority: Major
>
> In [FLIP-24] and  
> [FLIP-91|https://cwiki.apache.org/confluence/display/FLINK/FLIP-91%3A+Support+SQL+Gateway#FLIP91:SupportSQLGateway],
>  the SQL client was designed to has two mode: embedded and remote. Currently 
> only embedded mode has been implemented, so we need to implement the remote 
> mode.



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Commented] (FLINK-29471) Create a flink-connector-parent pom

2022-09-30 Thread Chesnay Schepler (Jira)


[ 
https://issues.apache.org/jira/browse/FLINK-29471?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17611467#comment-17611467
 ] 

Chesnay Schepler commented on FLINK-29471:
--

elasticsearch-main: 6e30d5d63d395b2f731418c34f5838231dcab6b8

> Create a flink-connector-parent pom
> ---
>
> Key: FLINK-29471
> URL: https://issues.apache.org/jira/browse/FLINK-29471
> Project: Flink
>  Issue Type: Sub-task
>  Components: Build System
>Reporter: Chesnay Schepler
>Assignee: Chesnay Schepler
>Priority: Major
>  Labels: pull-request-available
> Fix For: elasticsearch-3.0.0
>
>
> Create a shared parent pom for connectors, reducing the overhead of creating 
> new repos and easing plugin maintenance.



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[GitHub] [flink-connector-rabbitmq] zentol commented on pull request #1: [FLINK-20628] RabbitMQ Connector using FLIP-27 Source API

2022-09-30 Thread GitBox


zentol commented on PR #1:
URL: 
https://github.com/apache/flink-connector-rabbitmq/pull/1#issuecomment-1263341587

   You can avoid a lot of boilerplate by using the preliminary 
flink-connector-parent pom as shown here: 
https://github.com/apache/flink-connector-elasticsearch/commit/6e30d5d63d395b2f731418c34f5838231dcab6b8


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



[jira] [Closed] (FLINK-29372) Add a suffix to keys that violate YAML spec

2022-09-30 Thread Chesnay Schepler (Jira)


 [ 
https://issues.apache.org/jira/browse/FLINK-29372?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Chesnay Schepler closed FLINK-29372.

Resolution: Fixed

> Add a suffix to keys that violate YAML spec
> ---
>
> Key: FLINK-29372
> URL: https://issues.apache.org/jira/browse/FLINK-29372
> Project: Flink
>  Issue Type: Sub-task
>  Components: Runtime / Configuration
>Reporter: Chesnay Schepler
>Assignee: Chesnay Schepler
>Priority: Major
>  Labels: pull-request-available
> Fix For: 1.17.0
>
>
> We have a few options where the key is a prefix of other options (e.g., 
> {{high-availability}} and {{high-availability.cluster-id}}.
> Add a suffix to these options and keep the old key as deprecated.



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[GitHub] [flink-connector-elasticsearch] zentol merged pull request #40: [FLINK-29471] Use connector parent pom

2022-09-30 Thread GitBox


zentol merged PR #40:
URL: https://github.com/apache/flink-connector-elasticsearch/pull/40


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



[jira] [Updated] (FLINK-29486) Enable SQL Client to Connect SQL Gateway in Remote Mode

2022-09-30 Thread yuzelin (Jira)


 [ 
https://issues.apache.org/jira/browse/FLINK-29486?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

yuzelin updated FLINK-29486:

Description: In 
[FLIP-24|https://cwiki.apache.org/confluence/display/FLINK/FLIP-24%3A+SQL+Client]
 and  
[FLIP-91|https://cwiki.apache.org/confluence/display/FLINK/FLIP-91%3A+Support+SQL+Gateway#FLIP91:SupportSQLGateway],
 the SQL client was designed to has two mode: embedded and remote. Currently 
only embedded mode has been implemented, so we need to implement the remote 
mode.  (was: In [FLIP-24] and  
[FLIP-91|https://cwiki.apache.org/confluence/display/FLINK/FLIP-91%3A+Support+SQL+Gateway#FLIP91:SupportSQLGateway],
 the SQL client was designed to has two mode: embedded and remote. Currently 
only embedded mode has been implemented, so we need to implement the remote 
mode.)

> Enable SQL Client to Connect SQL Gateway in Remote Mode
> ---
>
> Key: FLINK-29486
> URL: https://issues.apache.org/jira/browse/FLINK-29486
> Project: Flink
>  Issue Type: New Feature
>  Components: Table SQL / Client
>Affects Versions: 1.17.0
>Reporter: yuzelin
>Priority: Major
>
> In 
> [FLIP-24|https://cwiki.apache.org/confluence/display/FLINK/FLIP-24%3A+SQL+Client]
>  and  
> [FLIP-91|https://cwiki.apache.org/confluence/display/FLINK/FLIP-91%3A+Support+SQL+Gateway#FLIP91:SupportSQLGateway],
>  the SQL client was designed to has two mode: embedded and remote. Currently 
> only embedded mode has been implemented, so we need to implement the remote 
> mode.



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Updated] (FLINK-29486) Enable SQL Client to Connect SQL Gateway in Remote Mode

2022-09-30 Thread yuzelin (Jira)


 [ 
https://issues.apache.org/jira/browse/FLINK-29486?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

yuzelin updated FLINK-29486:

Description: In [FLIP-24|http://example.com] and  [FLIP-91|], the SQL 
client was designed to has two mode: embedded and remote. Currently only 
embedded mode has been implemented, so we need to implement the remote mode.

> Enable SQL Client to Connect SQL Gateway in Remote Mode
> ---
>
> Key: FLINK-29486
> URL: https://issues.apache.org/jira/browse/FLINK-29486
> Project: Flink
>  Issue Type: New Feature
>  Components: Table SQL / Client
>Affects Versions: 1.17.0
>Reporter: yuzelin
>Priority: Major
>
> In [FLIP-24|http://example.com] and  [FLIP-91|], the SQL client was designed 
> to has two mode: embedded and remote. Currently only embedded mode has been 
> implemented, so we need to implement the remote mode.



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Updated] (FLINK-29471) Create a flink-connector-parent pom

2022-09-30 Thread Chesnay Schepler (Jira)


 [ 
https://issues.apache.org/jira/browse/FLINK-29471?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Chesnay Schepler updated FLINK-29471:
-
Fix Version/s: elasticsearch-3.0.0

> Create a flink-connector-parent pom
> ---
>
> Key: FLINK-29471
> URL: https://issues.apache.org/jira/browse/FLINK-29471
> Project: Flink
>  Issue Type: Sub-task
>  Components: Build System
>Reporter: Chesnay Schepler
>Assignee: Chesnay Schepler
>Priority: Major
>  Labels: pull-request-available
> Fix For: elasticsearch-3.0.0
>
>
> Create a shared parent pom for connectors, reducing the overhead of creating 
> new repos and easing plugin maintenance.



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Created] (FLINK-29486) Enable SQL Client to Connect SQL Gateway in Remote Mode

2022-09-30 Thread yuzelin (Jira)
yuzelin created FLINK-29486:
---

 Summary: Enable SQL Client to Connect SQL Gateway in Remote Mode
 Key: FLINK-29486
 URL: https://issues.apache.org/jira/browse/FLINK-29486
 Project: Flink
  Issue Type: New Feature
  Components: Table SQL / Client
Affects Versions: 1.17.0
Reporter: yuzelin






--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Created] (FLINK-29485) TypeSerializerUpgradeTestBase doesn't use FlinkVersions.current()

2022-09-30 Thread Matthias Pohl (Jira)
Matthias Pohl created FLINK-29485:
-

 Summary: TypeSerializerUpgradeTestBase doesn't use 
FlinkVersions.current()
 Key: FLINK-29485
 URL: https://issues.apache.org/jira/browse/FLINK-29485
 Project: Flink
  Issue Type: Bug
  Components: Tests
Affects Versions: 1.16.0, 1.17.0
Reporter: Matthias Pohl


{{TypeSerializerUpgradeTestBase}} still refers to 1.15 as the current version. 
We could use {{FlinkVersions.current()}} instead to avoid running into this 
issue again for future major updates.

I didn't check other occurrences of {{FlinkVersions}}. It should be verified as 
part of this Jira issue that we don't have the same issue in other locations as 
well.



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Updated] (FLINK-29350) Add a section for moving planner jar in Hive dependencies page

2022-09-30 Thread Huang Xingbo (Jira)


 [ 
https://issues.apache.org/jira/browse/FLINK-29350?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Huang Xingbo updated FLINK-29350:
-
Fix Version/s: 1.17.0
   (was: 1.16.0)

> Add a section for moving planner jar in Hive dependencies page
> --
>
> Key: FLINK-29350
> URL: https://issues.apache.org/jira/browse/FLINK-29350
> Project: Flink
>  Issue Type: Sub-task
>  Components: Connectors / Hive, Documentation
>Affects Versions: 1.16.0
>Reporter: luoyuxia
>Assignee: luoyuxia
>Priority: Major
>  Labels: pull-request-available
> Fix For: 1.17.0
>
>




--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Updated] (FLINK-28071) Support missing built-in functions in Table API

2022-09-30 Thread Huang Xingbo (Jira)


 [ 
https://issues.apache.org/jira/browse/FLINK-28071?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Huang Xingbo updated FLINK-28071:
-
Fix Version/s: 1.17.0
   (was: 1.16.0)

> Support missing built-in functions in Table API
> ---
>
> Key: FLINK-28071
> URL: https://issues.apache.org/jira/browse/FLINK-28071
> Project: Flink
>  Issue Type: Sub-task
>  Components: Table SQL / API
>Reporter: Dian Fu
>Assignee: Luning Wang
>Priority: Major
> Fix For: 1.17.0
>
>
> There are many built-in functions are not supported. See 
> https://nightlies.apache.org/flink/flink-docs-master/docs/dev/table/functions/systemfunctions/
>  for more details. There are two columns for each built-in function: *SQL 
> Function* and *Table Function*, if a function is not supported in *Table 
> API*, the *Table Function* column is documented as *N/A*. We need to evaluate 
> each of these functions to ensure that they could be used in both SQL and 
> Table API.



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Updated] (FLINK-27467) Remove CliFrontendTestBase

2022-09-30 Thread Huang Xingbo (Jira)


 [ 
https://issues.apache.org/jira/browse/FLINK-27467?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Huang Xingbo updated FLINK-27467:
-
Fix Version/s: 1.17.0
   (was: 1.16.0)

> Remove CliFrontendTestBase
> --
>
> Key: FLINK-27467
> URL: https://issues.apache.org/jira/browse/FLINK-27467
> Project: Flink
>  Issue Type: Technical Debt
>  Components: Command Line Client, Tests
>Reporter: Chesnay Schepler
>Assignee: Chesnay Schepler
>Priority: Major
>  Labels: pull-request-available, stale-assigned
> Fix For: 1.17.0
>
>
> The class isn't really providing any value.



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Updated] (FLINK-29372) Add a suffix to keys that violate YAML spec

2022-09-30 Thread Huang Xingbo (Jira)


 [ 
https://issues.apache.org/jira/browse/FLINK-29372?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Huang Xingbo updated FLINK-29372:
-
Fix Version/s: 1.17.0
   (was: 1.16.0)

> Add a suffix to keys that violate YAML spec
> ---
>
> Key: FLINK-29372
> URL: https://issues.apache.org/jira/browse/FLINK-29372
> Project: Flink
>  Issue Type: Sub-task
>  Components: Runtime / Configuration
>Reporter: Chesnay Schepler
>Assignee: Chesnay Schepler
>Priority: Major
>  Labels: pull-request-available
> Fix For: 1.17.0
>
>
> We have a few options where the key is a prefix of other options (e.g., 
> {{high-availability}} and {{high-availability.cluster-id}}.
> Add a suffix to these options and keep the old key as deprecated.



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Updated] (FLINK-29430) Sanity check in InternalKeyContextImpl#setCurrentKeyGroupIndex

2022-09-30 Thread Huang Xingbo (Jira)


 [ 
https://issues.apache.org/jira/browse/FLINK-29430?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Huang Xingbo updated FLINK-29430:
-
Fix Version/s: 1.17.0
   (was: 1.16.0)

> Sanity check in InternalKeyContextImpl#setCurrentKeyGroupIndex
> --
>
> Key: FLINK-29430
> URL: https://issues.apache.org/jira/browse/FLINK-29430
> Project: Flink
>  Issue Type: Bug
>  Components: Runtime / State Backends
>Affects Versions: 1.15.2
>Reporter: Zakelly Lan
>Assignee: Zakelly Lan
>Priority: Major
> Fix For: 1.17.0
>
>
> Currently the HeapStateBackend check whether the current key group index is a 
> valid one while the RocksDBStateBackend will not. When using 
> HeapStateBackend, if the user uses a non-deterministic shuffle key, an 
> exception is thrown as follows:
>  
> {code:java}
> java.lang.IllegalArgumentException: Key group 84 is not in 
> KeyGroupRange{startKeyGroup=32, endKeyGroup=63}. Unless you're directly using 
> low level state access APIs, this is most likely caused by non-deterministic 
> shuffle key (hashCode and equals implementation).
>     at 
> org.apache.flink.runtime.state.KeyGroupRangeOffsets.newIllegalKeyGroupException(KeyGroupRangeOffsets.java:37)
>     at 
> org.apache.flink.runtime.state.heap.StateTable.getMapForKeyGroup(StateTable.java:305)
>     at org.apache.flink.runtime.state.heap.StateTable.get(StateTable.java:261)
>     at org.apache.flink.runtime.state.heap.StateTable.get(StateTable.java:143)
>     at 
> org.apache.flink.runtime.state.heap.HeapValueState.value(HeapValueState.java:72)
>     at 
> com.alibaba.ververica.flink.state.benchmark.wordcount.WordCount$MixedFlatMapper.flatMap(WordCount.java:169)
>     at 
> com.alibaba.ververica.flink.state.benchmark.wordcount.WordCount$MixedFlatMapper.flatMap(WordCount.java:160)
>     at 
> org.apache.flink.streaming.api.operators.StreamFlatMap.processElement(StreamFlatMap.java:47)
>     at 
> org.apache.flink.streaming.runtime.tasks.OneInputStreamTask$StreamTaskNetworkOutput.emitRecord(OneInputStreamTask.java:233)
>     at 
> org.apache.flink.streaming.runtime.io.AbstractStreamTaskNetworkInput.processElement(AbstractStreamTaskNetworkInput.java:135)
>     at 
> org.apache.flink.streaming.runtime.io.AbstractStreamTaskNetworkInput.emitNext(AbstractStreamTaskNetworkInput.java:106)
>     at 
> org.apache.flink.streaming.runtime.io.StreamOneInputProcessor.processInput(StreamOneInputProcessor.java:65)
>     at 
> org.apache.flink.streaming.runtime.tasks.StreamTask.processInput(StreamTask.java:526)
>     at 
> org.apache.flink.streaming.runtime.tasks.mailbox.MailboxProcessor.runMailboxLoop(MailboxProcessor.java:203)
>     at 
> org.apache.flink.streaming.runtime.tasks.StreamTask.runMailboxLoop(StreamTask.java:811)
>     at 
> org.apache.flink.streaming.runtime.tasks.StreamTask.invoke(StreamTask.java:760)
>     at 
> org.apache.flink.runtime.taskmanager.Task.runWithSystemExitMonitoring(Task.java:954)
>     at 
> org.apache.flink.runtime.taskmanager.Task.restoreAndInvoke(Task.java:933)
>     at org.apache.flink.runtime.taskmanager.Task.doRun(Task.java:746)
>     at org.apache.flink.runtime.taskmanager.Task.run(Task.java:568)
>     at java.lang.Thread.run(Thread.java:750)
>  {code}
> However, the RocksDBStateBackend will run without an exception. The wrong key 
> group index will cause a state correctness problem, so it is better to do a 
> check in {_}InternalKeyContextImpl#{_}{_}setCurrentKeyGroupIndex{_}, and 
> throw an exception immediately.
>  



--
This message was sent by Atlassian Jira
(v8.20.10#820010)


[jira] [Updated] (FLINK-27432) Replace Time with Duration in TaskSlotTable

2022-09-30 Thread Huang Xingbo (Jira)


 [ 
https://issues.apache.org/jira/browse/FLINK-27432?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Huang Xingbo updated FLINK-27432:
-
Fix Version/s: 1.17.0
   (was: 1.16.0)

> Replace Time with Duration in TaskSlotTable
> ---
>
> Key: FLINK-27432
> URL: https://issues.apache.org/jira/browse/FLINK-27432
> Project: Flink
>  Issue Type: Sub-task
>  Components: Runtime / Coordination
>Reporter: Chesnay Schepler
>Assignee: Chesnay Schepler
>Priority: Major
> Fix For: 1.17.0
>
>




--
This message was sent by Atlassian Jira
(v8.20.10#820010)


  1   2   3   4   >