[jira] [Created] (FLINK-21277) SQLClientSchemaRegistryITCase fails to download testcontainers/ryuk:0.3.0

2021-02-04 Thread Dawid Wysakowicz (Jira)
Dawid Wysakowicz created FLINK-21277:


 Summary: SQLClientSchemaRegistryITCase fails to download 
testcontainers/ryuk:0.3.0
 Key: FLINK-21277
 URL: https://issues.apache.org/jira/browse/FLINK-21277
 Project: Flink
  Issue Type: Bug
  Components: Tests
Affects Versions: 1.12.1, 1.13.0
Reporter: Dawid Wysakowicz
Assignee: Dawid Wysakowicz
 Fix For: 1.12.2, 1.13.0


Tests using testcontainers fail from time to time downloading required images. 
Most probably caused by: 
https://github.com/testcontainers/testcontainers-java/issues/3574

We should upgrade testcontainers to 1.15.1



--
This message was sent by Atlassian Jira
(v8.3.4#803005)


Re: [DISCUSS]FLIP-163: SQL Client Improvements

2021-02-04 Thread Jark Wu
Hi all,

Regarding "One Parser", I think it's not possible for now because Calcite
parser can't parse
special characters (e.g. "-") unless quoting them as string literals.
That's why the WITH option
key are string literals not identifiers.

SET table.exec.mini-batch.enabled = true and ADD JAR /local/my-home/test.jar
have the same
problems. That's why we propose two parser, one splits lines into multiple
statements and match special
command through regex which is light-weight, and delegate other statements
to the other parser which is Calcite parser.

Note: we should stick on the unquoted SET table.exec.mini-batch.enabled =
true syntax,
both for backward-compatibility and easy-to-use, and all the other systems
don't have quotes on the key.


Regarding "table.planner" vs "sql-client.planner",
if we want to use "table.planner", I think we should explain clearly what's
the scope it can be used in documentation.
Otherwise, there will be users complaining why the planner doesn't change
when setting the configuration on TableEnv.
Would be better throwing an exception to indicate users it's now allowed to
change planner after TableEnv is initialized.
However, it seems not easy to implement.

Best,
Jark

On Thu, 4 Feb 2021 at 15:49, godfrey he  wrote:

> Hi everyone,
>
> Regarding "table.planner" and "table.execution-mode"
> If we define that those two options are just used to initialize the
> TableEnvironment, +1 for introducing table options instead of sql-client
> options.
>
> Regarding "the sql client, we will maintain two parsers", I want to give
> more inputs:
> We want to introduce sql-gateway into the Flink project (see FLIP-24 &
> FLIP-91 for more info [1] [2]). In the "gateway" mode, the CLI client and
> the gateway service will communicate through Rest API. The " ADD JAR
> /local/path/jar " will be executed in the CLI client machine. So when we
> submit a sql file which contains multiple statements, the CLI client needs
> to pick out the "ADD JAR" line, and also statements need to be submitted or
> executed one by one to make sure the result is correct. The sql file may be
> look like:
>
> SET xxx=yyy;
> create table my_table ...;
> create table my_sink ...;
> ADD JAR /local/path/jar1;
> create function my_udf as comMyUdf;
> insert into my_sink select ..., my_udf(xx) from ...;
> REMOVE JAR /local/path/jar1;
> drop function my_udf;
> ADD JAR /local/path/jar2;
> create function my_udf as comMyUdf2;
> insert into my_sink select ..., my_udf(xx) from ...;
>
> The lines need to be splitted into multiple statements first in the CLI
> client, there are two approaches:
> 1. The CLI client depends on the sql-parser: the sql-parser splits the
> lines and tells which lines are "ADD JAR".
> pro: there is only one parser
> cons: It's a little heavy that the CLI client depends on the sql-parser,
> because the CLI client is just a simple tool which receives the user
> commands and displays the result. The non "ADD JAR" command will be parsed
> twice.
>
> 2. The CLI client splits the lines into multiple statements and finds the
> ADD JAR command through regex matching.
> pro: The CLI client is very light-weight.
> cons: there are two parsers.
>
> (personally, I prefer the second option)
>
> Regarding "SHOW or LIST JARS", I think we can support them both.
> For default dialect, we support SHOW JARS, but if we switch to hive
> dialect, LIST JARS is also supported.
>
>
> [1] https://cwiki.apache.org/confluence/display/FLINK/FLIP-24+-+SQL+Client
> [2]
>
> https://cwiki.apache.org/confluence/display/FLINK/FLIP-91%3A+Support+SQL+Client+Gateway
>
> Best,
> Godfrey
>
> Rui Li  于2021年2月4日周四 上午10:40写道:
>
> > Hi guys,
> >
> > Regarding #3 and #4, I agree SHOW JARS is more consistent with other
> > commands than LIST JARS. I don't have a strong opinion about REMOVE vs
> > DELETE though.
> >
> > While flink doesn't need to follow hive syntax, as far as I know, most
> > users who are requesting these features are previously hive users. So I
> > wonder whether we can support both LIST/SHOW JARS and REMOVE/DELETE JARS
> > as synonyms? It's just like lots of systems accept both EXIT and QUIT as
> > the command to terminate the program. So if that's not hard to achieve,
> and
> > will make users happier, I don't see a reason why we must choose one over
> > the other.
> >
> > On Wed, Feb 3, 2021 at 10:33 PM Timo Walther  wrote:
> >
> > > Hi everyone,
> > >
> > > some feedback regarding the open questions. Maybe we can discuss the
> > > `TableEnvironment.executeMultiSql` story offline to determine how we
> > > proceed with this in the near future.
> > >
> > > 1) "whether the table environment has the ability to update itself"
> > >
> > > Maybe there was some misunderstanding. I don't think that we should
> > > support `tEnv.getConfig.getConfiguration.setString("table.planner",
> > > "old")`. Instead I'm proposing to support
> > > `TableEnvironment.create(Configuration)` where planner and execution
> > > mode are read immediately and

[jira] [Created] (FLINK-21278) NullpointExecption error is reported when using the evictor method to filter the data before the window calculation

2021-02-04 Thread HunterHunter (Jira)
HunterHunter created FLINK-21278:


 Summary: NullpointExecption error is reported when using the 
evictor method to filter the data before the window calculation
 Key: FLINK-21278
 URL: https://issues.apache.org/jira/browse/FLINK-21278
 Project: Flink
  Issue Type: Bug
  Components: API / DataStream
Affects Versions: 1.12.1
Reporter: HunterHunter


When I use evictor() method to filter the data before a window is triggered, if 
there is no data that meets the conditions, a nullpointExecption error will be 
reported.

This problem occurs in the ReduceApplyWindowFunction.apply method.

So I think if there is no data to calculate whether it can not trigger the 
calculation, or judge whether it is null before transmitting the calculation 
result



--
This message was sent by Atlassian Jira
(v8.3.4#803005)


Re: [DISCUSS]FLIP-163: SQL Client Improvements

2021-02-04 Thread Ingo Bürk
Hi,

regarding the (un-)quoted question, compatibility is of course an important
argument, but in terms of consistency I'd find it a bit surprising that
WITH handles it differently than SET, and I wonder if that could cause
friction for developers when writing their SQL.


Regards
Ingo

On Thu, Feb 4, 2021 at 9:38 AM Jark Wu  wrote:

> Hi all,
>
> Regarding "One Parser", I think it's not possible for now because Calcite
> parser can't parse
> special characters (e.g. "-") unless quoting them as string literals.
> That's why the WITH option
> key are string literals not identifiers.
>
> SET table.exec.mini-batch.enabled = true and ADD JAR
> /local/my-home/test.jar
> have the same
> problems. That's why we propose two parser, one splits lines into multiple
> statements and match special
> command through regex which is light-weight, and delegate other statements
> to the other parser which is Calcite parser.
>
> Note: we should stick on the unquoted SET table.exec.mini-batch.enabled =
> true syntax,
> both for backward-compatibility and easy-to-use, and all the other systems
> don't have quotes on the key.
>
>
> Regarding "table.planner" vs "sql-client.planner",
> if we want to use "table.planner", I think we should explain clearly what's
> the scope it can be used in documentation.
> Otherwise, there will be users complaining why the planner doesn't change
> when setting the configuration on TableEnv.
> Would be better throwing an exception to indicate users it's now allowed to
> change planner after TableEnv is initialized.
> However, it seems not easy to implement.
>
> Best,
> Jark
>
> On Thu, 4 Feb 2021 at 15:49, godfrey he  wrote:
>
> > Hi everyone,
> >
> > Regarding "table.planner" and "table.execution-mode"
> > If we define that those two options are just used to initialize the
> > TableEnvironment, +1 for introducing table options instead of sql-client
> > options.
> >
> > Regarding "the sql client, we will maintain two parsers", I want to give
> > more inputs:
> > We want to introduce sql-gateway into the Flink project (see FLIP-24 &
> > FLIP-91 for more info [1] [2]). In the "gateway" mode, the CLI client and
> > the gateway service will communicate through Rest API. The " ADD JAR
> > /local/path/jar " will be executed in the CLI client machine. So when we
> > submit a sql file which contains multiple statements, the CLI client
> needs
> > to pick out the "ADD JAR" line, and also statements need to be submitted
> or
> > executed one by one to make sure the result is correct. The sql file may
> be
> > look like:
> >
> > SET xxx=yyy;
> > create table my_table ...;
> > create table my_sink ...;
> > ADD JAR /local/path/jar1;
> > create function my_udf as comMyUdf;
> > insert into my_sink select ..., my_udf(xx) from ...;
> > REMOVE JAR /local/path/jar1;
> > drop function my_udf;
> > ADD JAR /local/path/jar2;
> > create function my_udf as comMyUdf2;
> > insert into my_sink select ..., my_udf(xx) from ...;
> >
> > The lines need to be splitted into multiple statements first in the CLI
> > client, there are two approaches:
> > 1. The CLI client depends on the sql-parser: the sql-parser splits the
> > lines and tells which lines are "ADD JAR".
> > pro: there is only one parser
> > cons: It's a little heavy that the CLI client depends on the sql-parser,
> > because the CLI client is just a simple tool which receives the user
> > commands and displays the result. The non "ADD JAR" command will be
> parsed
> > twice.
> >
> > 2. The CLI client splits the lines into multiple statements and finds the
> > ADD JAR command through regex matching.
> > pro: The CLI client is very light-weight.
> > cons: there are two parsers.
> >
> > (personally, I prefer the second option)
> >
> > Regarding "SHOW or LIST JARS", I think we can support them both.
> > For default dialect, we support SHOW JARS, but if we switch to hive
> > dialect, LIST JARS is also supported.
> >
> >
> > [1]
> https://cwiki.apache.org/confluence/display/FLINK/FLIP-24+-+SQL+Client
> > [2]
> >
> >
> https://cwiki.apache.org/confluence/display/FLINK/FLIP-91%3A+Support+SQL+Client+Gateway
> >
> > Best,
> > Godfrey
> >
> > Rui Li  于2021年2月4日周四 上午10:40写道:
> >
> > > Hi guys,
> > >
> > > Regarding #3 and #4, I agree SHOW JARS is more consistent with other
> > > commands than LIST JARS. I don't have a strong opinion about REMOVE vs
> > > DELETE though.
> > >
> > > While flink doesn't need to follow hive syntax, as far as I know, most
> > > users who are requesting these features are previously hive users. So I
> > > wonder whether we can support both LIST/SHOW JARS and REMOVE/DELETE
> JARS
> > > as synonyms? It's just like lots of systems accept both EXIT and QUIT
> as
> > > the command to terminate the program. So if that's not hard to achieve,
> > and
> > > will make users happier, I don't see a reason why we must choose one
> over
> > > the other.
> > >
> > > On Wed, Feb 3, 2021 at 10:33 PM Timo Walther 
> wrote:
> > >
> > > > Hi everyone,
> > > >
> > 

Re: [VOTE] FLIP-160: Declarative scheduler

2021-02-04 Thread Till Rohrmann
+1 (binding)

Cheers,
Till

On Wed, Feb 3, 2021 at 5:00 AM Yu Li  wrote:

> +1 (binding)
>
> Best Regards,
> Yu
>
>
> On Mon, 1 Feb 2021 at 17:21, Matthias Pohl  wrote:
>
> > +1 (non-binding)
> >
> > Thanks,
> > Matthias
> >
> > On Mon, Feb 1, 2021 at 4:22 AM Zhu Zhu  wrote:
> >
> > > +1 (binding)
> > >
> > > Thanks,
> > > Zhu
> > >
> > > Yang Wang  于2021年2月1日周一 上午11:04写道:
> > >
> > > > +1 (non-binding)
> > > >
> > > > Best,
> > > > Yang
> > > >
> > > > Yangze Guo  于2021年2月1日周一 上午9:50写道:
> > > >
> > > > > +1 (non-binding)
> > > > >
> > > > > Best,
> > > > > Yangze Guo
> > > > >
> > > > > On Sat, Jan 30, 2021 at 8:40 AM Xintong Song <
> tonysong...@gmail.com>
> > > > > wrote:
> > > > > >
> > > > > > +1 (binding)
> > > > > >
> > > > > > Thank you~
> > > > > >
> > > > > > Xintong Song
> > > > > >
> > > > > >
> > > > > >
> > > > > > On Fri, Jan 29, 2021 at 10:41 PM Robert Metzger <
> > rmetz...@apache.org
> > > >
> > > > > wrote:
> > > > > >
> > > > > > > ... and thanks a lot for your work :) I'm really excited about
> > > > finally
> > > > > > > adding this feature to Flink!
> > > > > > >
> > > > > > >
> > > > > > > On Fri, Jan 29, 2021 at 3:40 PM Robert Metzger <
> > > rmetz...@apache.org>
> > > > > > > wrote:
> > > > > > >
> > > > > > > > +1 (binding)
> > > > > > > >
> > > > > > > > On Fri, Jan 29, 2021 at 3:23 PM Till Rohrmann <
> > > > trohrm...@apache.org>
> > > > > > > > wrote:
> > > > > > > >
> > > > > > > >> Hi all,
> > > > > > > >>
> > > > > > > >> since the discussion [1] about FLIP-160 [2] seems to have
> > > reached
> > > > a
> > > > > > > >> consensus, I'd like to start a formal vote for the FLIP.
> > > > > > > >>
> > > > > > > >> Please vote +1 to approve the FLIP, or -1 with a comment.
> The
> > > vote
> > > > > will
> > > > > > > be
> > > > > > > >> open at least until Wednesday, Feb 3rd.
> > > > > > > >>
> > > > > > > >> Cheers,
> > > > > > > >> Till
> > > > > > > >>
> > > > > > > >> [1]
> > > > > > > >>
> > > > > > > >>
> > > > > > >
> > > > >
> > > >
> > >
> >
> https://lists.apache.org/thread.html/r604a01f739639e2a5f093fbe7894c172125530332747ecf6990a6ce4%40%3Cdev.flink.apache.org%3E
> > > > > > > >> [2]
> > > > > > > >>
> > > > > > > >>
> > > > > > >
> > > > >
> > > >
> > >
> >
> https://cwiki.apache.org/confluence/display/FLINK/FLIP-160%3A+Declarative+Scheduler
> > > > > > > >>
> > > > > > > >
> > > > > > >
> > > > >
> > > >
> >
>


[RESULT][VOTE] FLIP-160: Declarative scheduler

2021-02-04 Thread Till Rohrmann
Hi everyone,

The voting time for FLIP-160: Declarative scheduler [1] has passed. I'm
closing the vote now.

There were 8 +1 votes, 5 of which are binding:

- Robert Metzger (binding)
- Xintong Song (binding)
- Yangze (non-binding)
- Yang Wang (non-binding)
- Zhu Zhu (binding)
- Matthias Pohl (non-binding)
- Yu Li (binding)
- Till Rohrmann (binding)

There were no -1 votes.

Thus, FLIP-160 has been accepted.

Thanks everyone for joining the discussion and giving feedback!

[1]
https://lists.apache.org/thread.html/r5476d63b718bed70784b0d91ef03d28f4c4f6cf4cbe8760c7a7d5d73%40%3Cdev.flink.apache.org%3E

Cheers,
Till


Re: [DISCUSS]FLIP-163: SQL Client Improvements

2021-02-04 Thread Jark Wu
Hi Ingo,

Since we have supported the WITH syntax and SET command since v1.9 [1][2],
and
we have never received such complaints, I think it's fine for such
differences.

Besides, the TBLPROPERTIES clause of CREATE TABLE in Hive also requires
string literal keys[3],
and the SET = doesn't allow quoted keys [4].

Best,
Jark

[1]:
https://ci.apache.org/projects/flink/flink-docs-release-1.9/dev/table/connect.html
[2]:
https://ci.apache.org/projects/flink/flink-docs-release-1.9/dev/table/sqlClient.html#running-sql-queries
[3]: https://cwiki.apache.org/confluence/display/Hive/LanguageManual+DDL
[4]: https://cwiki.apache.org/confluence/display/Hive/LanguageManual+Cli
(search "set mapred.reduce.tasks=32")

On Thu, 4 Feb 2021 at 17:09, Ingo Bürk  wrote:

> Hi,
>
> regarding the (un-)quoted question, compatibility is of course an important
> argument, but in terms of consistency I'd find it a bit surprising that
> WITH handles it differently than SET, and I wonder if that could cause
> friction for developers when writing their SQL.
>
>
> Regards
> Ingo
>
> On Thu, Feb 4, 2021 at 9:38 AM Jark Wu  wrote:
>
> > Hi all,
> >
> > Regarding "One Parser", I think it's not possible for now because Calcite
> > parser can't parse
> > special characters (e.g. "-") unless quoting them as string literals.
> > That's why the WITH option
> > key are string literals not identifiers.
> >
> > SET table.exec.mini-batch.enabled = true and ADD JAR
> > /local/my-home/test.jar
> > have the same
> > problems. That's why we propose two parser, one splits lines into
> multiple
> > statements and match special
> > command through regex which is light-weight, and delegate other
> statements
> > to the other parser which is Calcite parser.
> >
> > Note: we should stick on the unquoted SET table.exec.mini-batch.enabled =
> > true syntax,
> > both for backward-compatibility and easy-to-use, and all the other
> systems
> > don't have quotes on the key.
> >
> >
> > Regarding "table.planner" vs "sql-client.planner",
> > if we want to use "table.planner", I think we should explain clearly
> what's
> > the scope it can be used in documentation.
> > Otherwise, there will be users complaining why the planner doesn't change
> > when setting the configuration on TableEnv.
> > Would be better throwing an exception to indicate users it's now allowed
> to
> > change planner after TableEnv is initialized.
> > However, it seems not easy to implement.
> >
> > Best,
> > Jark
> >
> > On Thu, 4 Feb 2021 at 15:49, godfrey he  wrote:
> >
> > > Hi everyone,
> > >
> > > Regarding "table.planner" and "table.execution-mode"
> > > If we define that those two options are just used to initialize the
> > > TableEnvironment, +1 for introducing table options instead of
> sql-client
> > > options.
> > >
> > > Regarding "the sql client, we will maintain two parsers", I want to
> give
> > > more inputs:
> > > We want to introduce sql-gateway into the Flink project (see FLIP-24 &
> > > FLIP-91 for more info [1] [2]). In the "gateway" mode, the CLI client
> and
> > > the gateway service will communicate through Rest API. The " ADD JAR
> > > /local/path/jar " will be executed in the CLI client machine. So when
> we
> > > submit a sql file which contains multiple statements, the CLI client
> > needs
> > > to pick out the "ADD JAR" line, and also statements need to be
> submitted
> > or
> > > executed one by one to make sure the result is correct. The sql file
> may
> > be
> > > look like:
> > >
> > > SET xxx=yyy;
> > > create table my_table ...;
> > > create table my_sink ...;
> > > ADD JAR /local/path/jar1;
> > > create function my_udf as comMyUdf;
> > > insert into my_sink select ..., my_udf(xx) from ...;
> > > REMOVE JAR /local/path/jar1;
> > > drop function my_udf;
> > > ADD JAR /local/path/jar2;
> > > create function my_udf as comMyUdf2;
> > > insert into my_sink select ..., my_udf(xx) from ...;
> > >
> > > The lines need to be splitted into multiple statements first in the CLI
> > > client, there are two approaches:
> > > 1. The CLI client depends on the sql-parser: the sql-parser splits the
> > > lines and tells which lines are "ADD JAR".
> > > pro: there is only one parser
> > > cons: It's a little heavy that the CLI client depends on the
> sql-parser,
> > > because the CLI client is just a simple tool which receives the user
> > > commands and displays the result. The non "ADD JAR" command will be
> > parsed
> > > twice.
> > >
> > > 2. The CLI client splits the lines into multiple statements and finds
> the
> > > ADD JAR command through regex matching.
> > > pro: The CLI client is very light-weight.
> > > cons: there are two parsers.
> > >
> > > (personally, I prefer the second option)
> > >
> > > Regarding "SHOW or LIST JARS", I think we can support them both.
> > > For default dialect, we support SHOW JARS, but if we switch to hive
> > > dialect, LIST JARS is also supported.
> > >
> > >
> > > [1]
> > https://cwiki.apache.org/confluence/display/FLINK/FLIP-24+-+SQL+C

[jira] [Created] (FLINK-21279) flink on yarn sink nothing

2021-02-04 Thread Spongebob (Jira)
Spongebob created FLINK-21279:
-

 Summary: flink on yarn sink nothing
 Key: FLINK-21279
 URL: https://issues.apache.org/jira/browse/FLINK-21279
 Project: Flink
  Issue Type: Bug
  Components: Table SQL / API
Affects Versions: 1.12.1
 Environment: flink: 1.12.1

hive: 3.1.2
Reporter: Spongebob
 Attachments: yarn.log

Here's the data chain of the flink application:
 # read from HDFS file via ExecutionEnvironment, get Dataset
 # collect the Dataset into Seq object.
 # transform Seq object into multi ArrayBuffer[Expression] objects.
 # create TableObjects from ArrayBuffers using `fromValues`
 # create catalog views from TableObjects
 # sink into hive table from catalog views.

It is all normal until step 6. And it runs inconsistently on local IDE and yarn 
cluster. When I run the application on local IDE it cost all network buffer 
memory then turn into failed( actually the HDFS file size is less then 2MB, and 
I had set the parrallelism of tableEnv to 1. If I run the one sink of them only 
it can be run normally). And to the yarn cluster, there throws the exception 
`Job was submitted in detached mode. Results of job execution, such as 
accumulators, runtime, etc. are not available. Please make sure your program 
doesn't call an eager execution function [collect, print, printToErr, count]` 
but the application can run successfully however can not sink anything to hive, 
I find it does not request any slot while running.



--
This message was sent by Atlassian Jira
(v8.3.4#803005)


Re: [VOTE] FLIP-160: Declarative scheduler

2021-02-04 Thread Maximilian Michels

+1 (binding)

On 04.02.21 10:18, Till Rohrmann wrote:

+1 (binding)

Cheers,
Till

On Wed, Feb 3, 2021 at 5:00 AM Yu Li  wrote:


+1 (binding)

Best Regards,
Yu


On Mon, 1 Feb 2021 at 17:21, Matthias Pohl  wrote:


+1 (non-binding)

Thanks,
Matthias

On Mon, Feb 1, 2021 at 4:22 AM Zhu Zhu  wrote:


+1 (binding)

Thanks,
Zhu

Yang Wang  于2021年2月1日周一 上午11:04写道:


+1 (non-binding)

Best,
Yang

Yangze Guo  于2021年2月1日周一 上午9:50写道:


+1 (non-binding)

Best,
Yangze Guo

On Sat, Jan 30, 2021 at 8:40 AM Xintong Song <

tonysong...@gmail.com>

wrote:


+1 (binding)

Thank you~

Xintong Song



On Fri, Jan 29, 2021 at 10:41 PM Robert Metzger <

rmetz...@apache.org



wrote:



... and thanks a lot for your work :) I'm really excited about

finally

adding this feature to Flink!


On Fri, Jan 29, 2021 at 3:40 PM Robert Metzger <

rmetz...@apache.org>

wrote:


+1 (binding)

On Fri, Jan 29, 2021 at 3:23 PM Till Rohrmann <

trohrm...@apache.org>

wrote:


Hi all,

since the discussion [1] about FLIP-160 [2] seems to have

reached

a

consensus, I'd like to start a formal vote for the FLIP.

Please vote +1 to approve the FLIP, or -1 with a comment.

The

vote

will

be

open at least until Wednesday, Feb 3rd.

Cheers,
Till

[1]













https://lists.apache.org/thread.html/r604a01f739639e2a5f093fbe7894c172125530332747ecf6990a6ce4%40%3Cdev.flink.apache.org%3E

[2]













https://cwiki.apache.org/confluence/display/FLINK/FLIP-160%3A+Declarative+Scheduler


















Upgrade calcite version

2021-02-04 Thread 盛森林
Hi,
I want to upgrade calcite to 1.22 in the flink branch that fork from apache 
release 1.9. 
Can community give me some suggestion.

[jira] [Created] (FLINK-21280) Investigate a way to disable object reuse in StateFun

2021-02-04 Thread Igal Shilman (Jira)
Igal Shilman created FLINK-21280:


 Summary: Investigate a way to disable object reuse in StateFun
 Key: FLINK-21280
 URL: https://issues.apache.org/jira/browse/FLINK-21280
 Project: Flink
  Issue Type: Task
  Components: Stateful Functions
Reporter: Igal Shilman


Currently in StateFun, we require object reuse to be enabled. We are doing it 
because Flink sources can emit arbitrary objects, but the first occurrence of 
the user defined payload serializer happens in a IngressRouterOperator. This 
works due to the fact that this operator is chained to the user, and with 
object reuse, the source output is passed as reference (without any Flink 
serialization) to the IngressRouterOperator.

Unfortunately JdbcSink requires object reuse to be disabled, and therefore 
can't be used directly with StateFun.

This issue was first reported on the mailing list:

[http://apache-flink-user-mailing-list-archive.2336050.n4.nabble.com/Stateful-Functions-JDBC-Sink-Problems-td41265.html]

 

This issue is about exploring a way to remove the object reuse requirement.



--
This message was sent by Atlassian Jira
(v8.3.4#803005)


Re: Proposal to add Google Cloud Storage FileSystem with RecoverableWriter

2021-02-04 Thread Galen Warren
Hi Till -- this is in response to your message before about my proposal to
add GCS FileSystem/RecoverableWriter support. I was only subscribed to the
dev digest before, and so I didn't actually get an email I could reply to,
so sorry for the one-off email. I'm properly subscribed to the dev list now.

I've added this PR related to
this effort, and I referenced it from the existing Jira ticket
.

Please let me know what the next steps are.

Thanks,

Galen


[jira] [Created] (FLINK-21281) Support DELETE FROM for flink sql

2021-02-04 Thread Jun Zhang (Jira)
Jun Zhang created FLINK-21281:
-

 Summary: Support  DELETE FROM for flink sql
 Key: FLINK-21281
 URL: https://issues.apache.org/jira/browse/FLINK-21281
 Project: Flink
  Issue Type: Improvement
  Components: Table SQL / Client
Affects Versions: 1.12.1
Reporter: Jun Zhang
 Fix For: 1.13.0


Support DELETE FROM for flink sql,the syntax like this:
{code:java}
DELETE FROM tablename [WHERE expression]
{code}



--
This message was sent by Atlassian Jira
(v8.3.4#803005)


[jira] [Created] (FLINK-21282) Support UPDATE for flink sql

2021-02-04 Thread Jun Zhang (Jira)
Jun Zhang created FLINK-21282:
-

 Summary: Support  UPDATE for flink sql
 Key: FLINK-21282
 URL: https://issues.apache.org/jira/browse/FLINK-21282
 Project: Flink
  Issue Type: Improvement
  Components: Table SQL / Client
Affects Versions: 1.12.1
Reporter: Jun Zhang
 Fix For: 1.13.0


Support UPDATE for flink sql,the syntax like this:
{code:java}
UPDATE tablename SET column = value [, column = value ...] [WHERE expression]
{code}



--
This message was sent by Atlassian Jira
(v8.3.4#803005)


[jira] [Created] (FLINK-21283) Support sql extension for flink sql

2021-02-04 Thread Jun Zhang (Jira)
Jun Zhang created FLINK-21283:
-

 Summary: Support sql extension for flink sql 
 Key: FLINK-21283
 URL: https://issues.apache.org/jira/browse/FLINK-21283
 Project: Flink
  Issue Type: Improvement
  Components: Table SQL / API
Affects Versions: 1.12.1
Reporter: Jun Zhang
 Fix For: 1.13.0


I think we should add sql extension for flink sql so that users can customize 
sql parsing, sql optimization, etc. we can refer to [spark sql extension 
|https://issues.apache.org/jira/browse/SPARK-18127]



--
This message was sent by Atlassian Jira
(v8.3.4#803005)


[jira] [Created] (FLINK-21284) Non-deterministic UDF functions return different values

2021-02-04 Thread hehuiyuan (Jira)
hehuiyuan created FLINK-21284:
-

 Summary: Non-deterministic UDF functions return different values
 Key: FLINK-21284
 URL: https://issues.apache.org/jira/browse/FLINK-21284
 Project: Flink
  Issue Type: Bug
  Components: Table SQL / Planner
Reporter: hehuiyuan


Non-deterministic UDF functions is used mutiple times , the result is different.

 
{code:java}
tableEnv.registerFunction("sample", new SampleFunction());

Table tm = tableEnv.sqlQuery("select name, RAND_INTEGER(10) as sample , sex 
from myhive_staff");
tableEnv.registerTable("tmp", tm);


tableEnv.sqlUpdate("insert into sinktable select * from tmp  where sample >= 
8");




// UDF函数
public class SampleFunction extends ScalarFunction {
  public int eval(int pvid) {
int a = (int) (Math.random() * 10);
System.out.println("" + a );
return a;
  }
}{code}
 

Sample udf function is used for `RAND_INTEGER(10) as sample`  when sink,

which lead  to  inconsistent result.

 

 



--
This message was sent by Atlassian Jira
(v8.3.4#803005)


[jira] [Created] (FLINK-21285) Support MERGE INTO for flink sql

2021-02-04 Thread Jun Zhang (Jira)
Jun Zhang created FLINK-21285:
-

 Summary: Support  MERGE INTO for flink sql
 Key: FLINK-21285
 URL: https://issues.apache.org/jira/browse/FLINK-21285
 Project: Flink
  Issue Type: Improvement
  Components: Table SQL / API
Affects Versions: 1.12.1
Reporter: Jun Zhang
 Fix For: 1.13.0


Support MERGE INTO for flink sql,refer to hive syntax:
{code:java}
MERGE INTO  AS T USING  AS S
ON 
WHEN MATCHED [AND ] THEN UPDATE SET 
WHEN MATCHED [AND ] THEN DELETE
WHEN NOT MATCHED [AND ] THEN INSERT VALUES
{code}



--
This message was sent by Atlassian Jira
(v8.3.4#803005)


[jira] [Created] (FLINK-21286) Support BUCKET for flink sql CREATE TABLE

2021-02-04 Thread Jun Zhang (Jira)
Jun Zhang created FLINK-21286:
-

 Summary: Support  BUCKET for flink sql CREATE TABLE
 Key: FLINK-21286
 URL: https://issues.apache.org/jira/browse/FLINK-21286
 Project: Flink
  Issue Type: Improvement
  Components: Table SQL / API
Affects Versions: 1.12.1
Reporter: Jun Zhang
 Fix For: 1.13.0


Support BUCKET for flink CREATE TABLE : refer to hive syntax
{code:java}
 [CLUSTERED BY (col_name, col_name, ...) [SORTED BY (col_name [ASC|DESC], ...)] 
INTO num_buckets BUCKETS]
{code}



--
This message was sent by Atlassian Jira
(v8.3.4#803005)


Re: [DISCUSS]FLIP-163: SQL Client Improvements

2021-02-04 Thread Jark Wu
Hi all,

After an offline discussion with Timo and Kurt, we have reached some
consensus.
Please correct me if I am wrong or missed anything.

1) We will introduce "table.planner" and "table.execution-mode" instead of
"sql-client" prefix,
and add `TableEnvironment.create(Configuration)` interface. These 2 options
can only be used
for tableEnv initialization. If used after initialization, Flink should
throw an exception. We may can
support dynamic switch the planner in the future.

2) We will have only one parser,
i.e. org.apache.flink.table.delegation.Parser. It accepts a string
statement, and returns a list of Operation. It will first use regex to
match some special statement,
 e.g. SET, ADD JAR, others will be delegated to the underlying Calcite
parser. The Parser can
have different implementations, e.g. HiveParser.

3) We only support ADD JAR, REMOVE JAR, SHOW JAR for Flink dialect. But we
can allow
DELETE JAR, LIST JAR in Hive dialect through HiveParser.

4) We don't have a conclusion for async/sync execution behavior yet.

Best,
Jark



On Thu, 4 Feb 2021 at 17:50, Jark Wu  wrote:

> Hi Ingo,
>
> Since we have supported the WITH syntax and SET command since v1.9 [1][2],
> and
> we have never received such complaints, I think it's fine for such
> differences.
>
> Besides, the TBLPROPERTIES clause of CREATE TABLE in Hive also requires
> string literal keys[3],
> and the SET = doesn't allow quoted keys [4].
>
> Best,
> Jark
>
> [1]:
> https://ci.apache.org/projects/flink/flink-docs-release-1.9/dev/table/connect.html
> [2]:
> https://ci.apache.org/projects/flink/flink-docs-release-1.9/dev/table/sqlClient.html#running-sql-queries
> [3]: https://cwiki.apache.org/confluence/display/Hive/LanguageManual+DDL
> [4]: https://cwiki.apache.org/confluence/display/Hive/LanguageManual+Cli
> (search "set mapred.reduce.tasks=32")
>
> On Thu, 4 Feb 2021 at 17:09, Ingo Bürk  wrote:
>
>> Hi,
>>
>> regarding the (un-)quoted question, compatibility is of course an
>> important
>> argument, but in terms of consistency I'd find it a bit surprising that
>> WITH handles it differently than SET, and I wonder if that could cause
>> friction for developers when writing their SQL.
>>
>>
>> Regards
>> Ingo
>>
>> On Thu, Feb 4, 2021 at 9:38 AM Jark Wu  wrote:
>>
>> > Hi all,
>> >
>> > Regarding "One Parser", I think it's not possible for now because
>> Calcite
>> > parser can't parse
>> > special characters (e.g. "-") unless quoting them as string literals.
>> > That's why the WITH option
>> > key are string literals not identifiers.
>> >
>> > SET table.exec.mini-batch.enabled = true and ADD JAR
>> > /local/my-home/test.jar
>> > have the same
>> > problems. That's why we propose two parser, one splits lines into
>> multiple
>> > statements and match special
>> > command through regex which is light-weight, and delegate other
>> statements
>> > to the other parser which is Calcite parser.
>> >
>> > Note: we should stick on the unquoted SET table.exec.mini-batch.enabled
>> =
>> > true syntax,
>> > both for backward-compatibility and easy-to-use, and all the other
>> systems
>> > don't have quotes on the key.
>> >
>> >
>> > Regarding "table.planner" vs "sql-client.planner",
>> > if we want to use "table.planner", I think we should explain clearly
>> what's
>> > the scope it can be used in documentation.
>> > Otherwise, there will be users complaining why the planner doesn't
>> change
>> > when setting the configuration on TableEnv.
>> > Would be better throwing an exception to indicate users it's now
>> allowed to
>> > change planner after TableEnv is initialized.
>> > However, it seems not easy to implement.
>> >
>> > Best,
>> > Jark
>> >
>> > On Thu, 4 Feb 2021 at 15:49, godfrey he  wrote:
>> >
>> > > Hi everyone,
>> > >
>> > > Regarding "table.planner" and "table.execution-mode"
>> > > If we define that those two options are just used to initialize the
>> > > TableEnvironment, +1 for introducing table options instead of
>> sql-client
>> > > options.
>> > >
>> > > Regarding "the sql client, we will maintain two parsers", I want to
>> give
>> > > more inputs:
>> > > We want to introduce sql-gateway into the Flink project (see FLIP-24 &
>> > > FLIP-91 for more info [1] [2]). In the "gateway" mode, the CLI client
>> and
>> > > the gateway service will communicate through Rest API. The " ADD JAR
>> > > /local/path/jar " will be executed in the CLI client machine. So when
>> we
>> > > submit a sql file which contains multiple statements, the CLI client
>> > needs
>> > > to pick out the "ADD JAR" line, and also statements need to be
>> submitted
>> > or
>> > > executed one by one to make sure the result is correct. The sql file
>> may
>> > be
>> > > look like:
>> > >
>> > > SET xxx=yyy;
>> > > create table my_table ...;
>> > > create table my_sink ...;
>> > > ADD JAR /local/path/jar1;
>> > > create function my_udf as comMyUdf;
>> > > insert into my_sink select ..., my_udf(xx) from ...;
>> > > REMOVE JAR /local/path/jar1;
>> > > drop funct

[jira] [Created] (FLINK-21287) Failed to build flink source code

2021-02-04 Thread Lei Qu (Jira)
Lei Qu created FLINK-21287:
--

 Summary: Failed to build flink source code
 Key: FLINK-21287
 URL: https://issues.apache.org/jira/browse/FLINK-21287
 Project: Flink
  Issue Type: Bug
  Components: Formats (JSON, Avro, Parquet, ORC, SequenceFile)
Affects Versions: 1.12.1
Reporter: Lei Qu


[ERROR] Failed to execute goal 
org.xolstice.maven.plugins:protobuf-maven-plugin:0.5.1:test-compile (default) 
on project flink-parquet_2.11: protoc did not exit cleanly. Review output for 
more information. -> [Help 1]
[ERROR]
[ERROR] To see the full stack trace of the errors, re-run Maven with the -e 
switch.
[ERROR] Re-run Maven using the -X switch to enable full debug logging.
[ERROR]
[ERROR] For more information about the errors and possible solutions, please 
read the following articles:
[ERROR] [Help 1] 
http://cwiki.apache.org/confluence/display/MAVEN/MojoFailureException
[ERROR]
[ERROR] After correcting the problems, you can resume the build with the command
[ERROR] mvn  -rf :flink-parquet_2.11



--
This message was sent by Atlassian Jira
(v8.3.4#803005)


[jira] [Created] (FLINK-21288) Support redundant task managers for fine grained resource management

2021-02-04 Thread Xintong Song (Jira)
Xintong Song created FLINK-21288:


 Summary: Support redundant task managers for fine grained resource 
management
 Key: FLINK-21288
 URL: https://issues.apache.org/jira/browse/FLINK-21288
 Project: Flink
  Issue Type: Sub-task
Reporter: Xintong Song






--
This message was sent by Atlassian Jira
(v8.3.4#803005)


[jira] [Created] (FLINK-21289) Application mode on kubernetes deployment support run PackagedProgram.main with pipeline.classpaths

2021-02-04 Thread Zhou Parker (Jira)
Zhou Parker created FLINK-21289:
---

 Summary: Application mode on kubernetes deployment support run 
PackagedProgram.main with pipeline.classpaths
 Key: FLINK-21289
 URL: https://issues.apache.org/jira/browse/FLINK-21289
 Project: Flink
  Issue Type: Improvement
  Components: Client / Job Submission, Deployment / Kubernetes
Affects Versions: 1.12.1, 1.11.2
 Environment: flink: 1.11

kubernetes: 1.15

 
Reporter: Zhou Parker
 Attachments: 0001-IMP.patch

我尝试将flink作业以application 
mode方式提交到kubernetes上运行。但程序的依赖包并不完全存在于local:///opt/flink/usrlib/.jar中。导致找不到类。

在yarn上可以工作,是因为我们用 {color:#FF}-C [http://|http:///] 
{color}的方式,让依赖可以被URLClassloader加载。

但我发现,当实验提交到kubernetes时,-C只会在 configmap/flink-conf.yaml 中生成一个pipeline.classpaths 
配置条目。我们的main函数可以执行,但是在加载外部依赖类的时候提示找不到类。

通过阅读源码,*我发现运行用户代码的类加载器实际并把 pipeline.classpaths 
中的条目加入候选URL*,这导致了无法加载类的情况。从源码中,我也发现,可以通过将依赖包放在usrlib目录下(默认的userClassPath)可以解决问题。但我们的依赖可能是动态的,不合适一次性打到镜像里面。

我提议可以改进这个过程,将pipeline.classpaths也加入到对应的类加载器。这个改动很小,我自己经过测试,可以完美解决问题。

 

I'm trying to submit flink job to kubernetes cluster with application mode, but 
throw ClassNotFoundException when dependency class is not shipped in kind of 
local:///opt/flink/usrlib/.jar.

This works on yarn, since we use {color:#FF}-C http://{color} command 
line style that let dependency class  can be load by URLClassloader.

But i figure out that not works on kubernetes. When submit to kubernetes 
cluster, -C is only shipped as item "pipeline.classpaths" in 
configmap/flink-conf.yaml。

After read the source code, *i find out that the Classloader launching the 
"main" entry of user code without consider add pipeline.classpaths into 
candidates URLs*. from source code, i also learn that we can ship the 
dependency jar in the usrlib dir to solve the problem. But failed for me, we 
are not _preferred_ to ship dependencies in image at compile time, since they 
are known dynamically in runtime

I proposed improving the process, let the Classloader consider usrlib as well 
as pipeline.classpaths, this is a quite little change. I test the solution and 
it works quite well

 

 



--
This message was sent by Atlassian Jira
(v8.3.4#803005)


Re: [DISCUSS]FLIP-150: Introduce Hybrid Source

2021-02-04 Thread Kezhu Wang
Hi all,

Thanks for starting the discussion!

I think there are differences between start-position and checkpoint-state.
Checkpoint states
are actually intermediate progress-capture in reading data. Source clients
don't care about it
in coding phase, there are purely implementation details. While
start-position, on the other side,
is part of public API, say, paths for FileSource/HiveSource,
OffsetsInitializer for KafkaSource.
Given that, `setStartState` is actually a reposition operation for next
source to start in job runtime.

Besides above, I think this separation of start-position and
checkpoint-state leads to smooth
migration from no-switchable source to switchable source and relaxes source
writers. Source writers
will not have to take into account of switchable-or-not in design phase,
but a postponable decision.
Later enhancement will not break anything. I would be relative hard to
convert start-position to
checkpoint-state in restoring without changing to checkpoint-state
structure and/or serializer:

* `Paths[]` to `PendingSplitsCheckpoint`.
* `OffsetsInitializer` to `KafkaSourceEnumState`.

This conversion should be implementation detail of next source, not
converter function in my opinion.

With separated start-position concept, I think the `StartStateT` in
proposal should be `Path[]` for
`FileSource`, `OffsetsInitializer` for `KafkaSource`. Seems like it should
belong to source more
than split enumerator.

If we do not reuse checkpoint-state as start-position, then this feature
requires supports from Flink
side(that says it is awkward/tangled to be used as a third party libraries):
* It needs a post-mortem position to delivery possible useful information
for next source.
* It needs to reposition next source, such as `FileSource`, `KafkaSource`,
etc.

Back to the proposal, I think it shows three joints:
1. Separated start-position for next source.
2. Separated end-position from preceding source.
3. Converter from end-position to start-position.

I think some or all of the three should be optional. Let me detail:
1. No configurable start-position. In this situation combination of above
three is a nop,
   and `HybridSource` is just a chain of start-position pre-configured
sources. Current design seems
   overkilling for this use case. This case could also be realized with
help from `InputSelectable`
   and `MultipleInputStreamOperator`, but I think it fits this feature
quite well.
2. I am wonder whether end-position is a must and how it could be useful
for end users in
   a generic-enough source, say `FileSource` ? I could imagine that some
sources may have embedded
   next-position for next source to start. For most generic sources, they
actually have no meaningful
   next-position for other sources. Then would it be too risky to coin this
to generic sources' type ?
   Or we are actually doing this solely to fit type requirement of
`HybridSource.addSource` ?
3. Is it possible for converter function to do blocking operations ? How to
respond to checkpoint
   request when switching split enumerators cross sources ? Does
end-position or start-position
   need to be stored in checkpoint state or not ?

Last, for the name `HybridSource`, would it be possible to use this feature
to switch/chain multiple
homogeneous sources ? Say:

* Two file sources with different formats.
* Migrate from one kafka cluster to another as @Thomas has already pointed
out.



On February 4, 2021 at 10:48:21, Thomas Weise (t...@apache.org) wrote:

Thanks for initiating this discussion and creating the proposal!

I would like to contribute to this effort. Has there been related activity
since the FLIP was created?

If not, I would like to start work on a PoC to validate the design.

Questions/comments:

There could be more use cases for a hybrid source beyond predefined
sequence that is fixed at job submission time. For example, the source
connector could be used to migrate from one external system to another
(like Kafka1 .. KafkaN - based on external trigger/discovery).

I agree with @Aljoscha Krettek  that it would be
preferable to solve this without special "switchable" interfaces and have
it work with any FLIP-27 source as is. Performing the switch using the
enumerator checkpoint appears viable (not proven though unless coded 😉).
The actual FLIP-27 source reader would need to signal to the
"HybridSourceReader" (HSR) that they are done and then the HSR would send
the switch event to the coordinator?

To further confirm my understanding:

The actual split type that flows between enumerator and reader would be
"HybridSourceSplit" and it would wrap the specific split (in the example
either HDFS or Kafka)?

Switching relies on the previous source's end position to be communicated
as start position to the next source. The position(s) can be exchanged
through the checkpoint state, but "HybridSplitEnumerator" still needs a way
to extract them from the actual enumerator. That could be done by the
enumerator checkpoint state mapping

[jira] [Created] (FLINK-21290) Support Projection push down for Window TVF

2021-02-04 Thread Jark Wu (Jira)
Jark Wu created FLINK-21290:
---

 Summary: Support Projection push down for Window TVF
 Key: FLINK-21290
 URL: https://issues.apache.org/jira/browse/FLINK-21290
 Project: Flink
  Issue Type: Sub-task
  Components: Table SQL / Planner
Reporter: Jark Wu


{code:scala}
  @Test
  def testTumble_ProjectionPushDown(): Unit = {
// TODO: [b, c, e, proctime] are never used, should be pruned
val sql =
  """
|SELECT
|   a,
|   window_start,
|   window_end,
|   count(*),
|   sum(d)
|FROM TABLE(TUMBLE(TABLE MyTable, DESCRIPTOR(rowtime), INTERVAL '15' 
MINUTE))
|GROUP BY a, window_start, window_end
  """.stripMargin
util.verifyRelPlan(sql)
  }
{code}

For the above test, currently we get the following plan:

{code}
Calc(select=[a, window_start, window_end, EXPR$3, EXPR$4])
+- WindowAggregate(groupBy=[a], window=[TUMBLE(time_col=[rowtime], size=[15 
min])], select=[a, COUNT(*) AS EXPR$3, SUM(d) AS EXPR$4, start('w$) AS 
window_start, end('w$) AS window_end])
   +- Exchange(distribution=[hash[a]])
  +- Calc(select=[a, d, rowtime])
 +- WatermarkAssigner(rowtime=[rowtime], watermark=[-(rowtime, 
1000:INTERVAL SECOND)])
+- Calc(select=[a, b, c, d, e, rowtime, PROCTIME() AS proctime])
   +- TableSourceScan(table=[[default_catalog, default_database, 
MyTable]], fields=[a, b, c, d, e, rowtime])
{code}

It should be able to prune fields and get the following plan:

{code}
Calc(select=[a, window_start, window_end, EXPR$3, EXPR$4])
+- WindowAggregate(groupBy=[a], window=[TUMBLE(time_col=[rowtime], size=[15 
min])], select=[a, COUNT(*) AS EXPR$3, SUM(d) AS EXPR$4, start('w$) AS 
window_start, end('w$) AS window_end])
   +- Exchange(distribution=[hash[a]])
 +- WatermarkAssigner(rowtime=[rowtime], watermark=[-(rowtime, 
1000:INTERVAL SECOND)])
   +- TableSourceScan(table=[[default_catalog, default_database, 
MyTable]], fields=[a, d, rowtime])
{code}

The reason is we didn't transpose Project and WindowTableFunction in logical 
phase. 

{code}
LogicalAggregate(group=[{0, 1, 2}], EXPR$3=[COUNT()], EXPR$4=[SUM($3)])
+- LogicalProject(a=[$0], window_start=[$7], window_end=[$8], d=[$3])
   +- LogicalTableFunctionScan(invocation=[TUMBLE($6, DESCRIPTOR($5), 
90:INTERVAL MINUTE)], rowType=[RecordType(INTEGER a, BIGINT b, 
VARCHAR(2147483647) c, DECIMAL(10, 3) d, BIGINT e, TIME ATTRIBUTE(ROWTIME) 
rowtime, TIME ATTRIBUTE(PROCTIME) proctime, TIMESTAMP(3) window_start, 
TIMESTAMP(3) window_end, TIME ATTRIBUTE(ROWTIME) window_time)])
  +- LogicalProject(a=[$0], b=[$1], c=[$2], d=[$3], e=[$4], rowtime=[$5], 
proctime=[$6])
 +- LogicalWatermarkAssigner(rowtime=[rowtime], watermark=[-($5, 
1000:INTERVAL SECOND)])
+- LogicalProject(a=[$0], b=[$1], c=[$2], d=[$3], e=[$4], 
rowtime=[$5], proctime=[PROCTIME()])
   +- LogicalTableScan(table=[[default_catalog, default_database, 
MyTable]])
{code}



--
This message was sent by Atlassian Jira
(v8.3.4#803005)


[jira] [Created] (FLINK-21291) FlinkKafka Consumer can't dynamic discover the partition update

2021-02-04 Thread zhangyunyun (Jira)
zhangyunyun created FLINK-21291:
---

 Summary: FlinkKafka Consumer can't dynamic discover the partition 
update
 Key: FLINK-21291
 URL: https://issues.apache.org/jira/browse/FLINK-21291
 Project: Flink
  Issue Type: Bug
  Components: Connectors / Kafka
Affects Versions: 1.11.2
Reporter: zhangyunyun


When start the job, occurs WARN log like below:
{code:java}
WARN  org.apache.kafka.clients.consumer.ConsumerConfig  - The configuration 
'flink.partition-discovery.interval-millis' was supplied but isn't a known 
config.

{code}
 

And I try to change the kafka partion with command, partition number from 3 to 4
{code:java}

./kafka-topics.sh --alter --zookeeper 10.0.10.21:15311 --topic STRUCTED_LOG 
--partitions 4
{code}
it dosen't work.

 

How can I do with this problem. Thanks a lot

 



--
This message was sent by Atlassian Jira
(v8.3.4#803005)