Hi Emarotti,
It sounds like you want to process your streams based on the event time. This
means, all your processing, windowing, timer and co are based on a timestamp
that is provided by individual events. You can read more about here:
https://nightlies.apache.org/flink/flink-docs-release-1.16/
Hi Flink-User!
I recently updated a Flink job from Flink version 1.13 to 1.15 (managed by
AWS). The Flink Job is written in Java.
I found out that the Kinesis Producer was deprecated in favour of Kinesis
Streams Sink [1]. When upgrading to the new sink I stumbled upon a problem
withe a custom
ing-java/src/main/java/org/apache/flink/streaming/runtime/operators/TimestampsAndWatermarksOperator.java#L120
>
> <https://github.com/apache/flink/blob/release-1.15/flink-streaming-java/src/main/java/org/apache/flink/streaming/runtime/operators/TimestampsAndWatermarksOperator.java#L120&
Hi there,
While still struggling with events and watermarks out of order after sorting
with a buffer process function (compare [1]) I tired to solve the issue by
assigning a new watermark after the mentioned sorting function.
The Flink docs [2] are not very certain about the impact of assigning
Background:
Using Flink v. 1.13.2 on AWS, with job parallelism of 4.
Ingress data from AWS Kinesis are not partitioned by the correct key according
to business logic. For that reason, events are repartitioned by using a
KeyedStream produced by calling keyBy(.) function providing the correct logi
ueues-from-and-to-apache-kafka-with-apache-flink/>
>
> Best regards,
>
> Martijn
>
>
> Op wo 3 aug. 2022 om 15:56 schreef Peter Schrott <mailto:pe...@bluerootlabs.io>>:
> Hi Flink Ppl!
>
> Working with Apache Flink v 1.13.2 on AWS with Kinesis as sou
Hi Flink Ppl!
Working with Apache Flink v 1.13.2 on AWS with Kinesis as source.
I have the requirement to drop certain events before they enter the jobs
pipeline. There are mainly 2 reasons:
1) Problems when it comes to deserializing the incoming bytes
2) Problems with the event itself, e.g. mis
Hi there!
I have a Flink Job (v 1.13.2, AWS managed) which reads from Kinesis (AWS
manger, 4 shards).
For reasons the shards are not partitioned properly (at the moment). So I
wanted to make use of Watermarks (BoundedOutOfOrdernessTimestampExtractor) and
the JobManagerWatermarkTracker to avoid
Hi Vikash,
Could you be more precise about the shared libraries? Is there any
documentation about this?
Thanks, Peter
On Tue, May 24, 2022 at 1:23 PM Vikash Dat wrote:
> Similar to agent Biao, Application mode is okay if you only have a single
> app, but when running multiple apps session mode
Hi Ghiy,
I am not quite sure about your actual problem, why the schema is not
generated as expected.
I also needed to work with the Kafka keys in the business logic, therefore
I found a way to deserialize and serialize the key along with the event
itself by overriding KafkaRecord[De]Serialization
Hi ChangZhuo,
By low level process function you mean the Data Streams API, right?
How to disable the Kafka metrics when creating a Kafka source / sink is
described here:
https://nightlies.apache.org/flink/flink-docs-release-1.15/docs/connectors/datastream/kafka/#additional-properties
The same pr
nd
> exclude specific problematic metrics; see the docs.)
> On 04/05/2022 13:36, Peter Schrott wrote:
>
> Allright! Thanks!
>
> I tried to dig a bit deeper and see if there is any workaround for that
> problem. I tried to switch off reporting the Kafka metrics, but I was not
>
every job that uses the new kafka connector.
>
> Thank you for debugging the issue!
>
> I will create a ticket.
>
> On 04/05/2022 12:24, Peter Schrott wrote:
>
> As the stracktrace says, class cast exception occurs here:
>
> https://github.com/apache/flink/blob/mast
https://github.com/apache/flink/commit/b367407d08b6dd69a52886a1c6232a9d8ee2ec0a#diff-bb47c4c2d77fd57da49a6cf5227d43ba352c2ea916776bdae92a7436dea50068
Is this a potential bug introduced in 1.15.0?
Best, Peter
On Wed, May 4, 2022 at 9:58 AM Peter Schrott wrote:
> Sorry for the spamming!
>
> Just aft
=/127.0.0.1:9200 remote=/127.0.0.1:50508]
For my defence: This jul - slf4j - logback setup is really nasty :O
Best, Peter
On Wed, May 4, 2022 at 9:47 AM Peter Schrott wrote:
> Hi Chesnay,
>
> Thanks for that support! Just for compilation: Running the "Problem-Job"
> locally a
cluster.sh), I do not have a certain
> url/port to access the taskmanager, right?
>
> If you configure a port range it should be as simple as curl
> localhost:.
> You can find the used port in the taskmanager logs.
> Or just try the first N ports in the range ;)
>
>
E
> appender.testlogger.target = SYSTEM_ERR
> appender.testlogger.layout.type = PatternLayout
> appender.testlogger.layout.pattern = %-4r [%t] %-5p %c %x - %m%n
>
> On 03/05/2022 10:41, ChangZhuo Chen (陳昌倬) wrote:
>
> On Tue, May 03, 2022 at 10:32:03AM +0200, Peter Schrott
Hi!
I also discovered problems with the PrometheusReporter on Flink 1.15.0,
coming from 1.14.4. I already consulted the mailing list:
https://lists.apache.org/thread/m8ohrfkrq1tqgq7lowr9p226z3yc0fgc
I have not found the underlying problem or a solution to it.
Actually, after re-checking, I see th
t; details on the investigation in FLINK-27354 [1] itself.
>
> Best,
> Matthias
>
> [1] https://issues.apache.org/jira/browse/FLINK-27354
>
> On Mon, Apr 25, 2022 at 2:00 PM Matthias Pohl
> wrote:
>
>> Thanks Peter, we're looking into it...
>>
>> On Mon
ptions.
>
> On 20/04/2022 18:45, Peter Schrott wrote:
>
> Hi kuweiha,
>
> Just to confirm, you tried with 1.15 - none of the rcs are working for me?
>
> This port is definitely free as it was already used on the same hosts with
> Flink 1.14.4. And as I said, when no job
Hi Flink-Users,
I am not sure if this does something to my cluster or not. But since
updating to Flink 1.15 (atm rc4) I find the following logs:
INFO: Registering job manager ab7db9ff0ebd26b3b89c3e2e56684...@akka.tcp://
fl...@flink-jobmanager-xxx.com:40015/user/rpc/jobmanager_2 for job
5566648d9b
of the host
> is being listened by the TaskManager process.
>
>
> 2022年4月20日 下午10:48,Peter Schrott 写道:
>
> Hi Flink-Users,
>
> After upgrading to Flink 1.15 (rc3) (coming from 1.14) I noticed that
> there is a problem with the metrics exposed through the Prometheu
Hi Flink-Users,
After upgrading to Flink 1.15 (rc3) (coming from 1.14) I noticed that there
is a problem with the metrics exposed through the PrometheusReporter.
It is configured as followed in the flink-config.yml:
metrics.reporters: prom
metrics.reporter.prom.class:
org.apache.flink.metrics.pro
Hi Flink Users,
Does anyone know what happened to the /status endpoint of a job?
Calling /jobs/0c39e6ce662379449e7f7f965ff1eee0/status gives me a 404.
Thanks & best, Peter
Hi there,
I just successfully upgraded our Flink cluster to 1.15.0 rc0 - also the
corresponding job is running on this version. Looks great so far!
In the Web UI I noticed some metrics are missing, especially "Records
Received", "Bytes Sent" and "Records Sent". Those were shown in v 1.14.4.
See a
unsubscribe
Hi people,
I found a workaround for that issue - which works at least for my use case.
The main idea was customizing
"org.apache.flink.formats.avro.registry.confluent.RegistryAvroFormatFactory"
such that the expected avro schema is not gained from the CREATE TABLE SQL
statement, rather than pa
gt; We will further improve the support of external types in the Table API
> type system in the near future.
>
> Regards,
> Timo
>
> On 20.10.21 15:51, Peter Schrott wrote:
> > Hi people!
> >
> > I was digging deeper this days and found the "root cause" o
Hi people!
I was digging deeper this days and found the "root cause" of the issue and the
difference between avro reading from files and avro reading from Kafka & SR.
plz see:
https://lists.apache.org/x/thread.html/r8ad7bd574f7dc4904139295c7de612a35438571c5b9caac673521d22@%3Cuser.flink.apache.o
Hi & thanks,
with your solution you are referring the the reported exception:
`Found my.type.avro.MyEnumType, expecting union`
I investigated on the "union" part and added "NOT NULL" to the SQL statement,
such that the attribute is NOT nullable on avro AND SQL. This actually "fixed"
the report
ence to a
> String, then change the stream to a table.
>
> Peter Schrott 于2021年10月18日周一 下午8:40写道:
>
>> Hi there,
>>
>> I have a Kafka topic where the schema of its values is defined by the
>> "MyRecord" record in the following Avro IDL and registered
izhi Weng wrote:
> Hi!
>
> You can call streamSource.processRecord to change the CharSequence to a
> String, then change the stream to a table.
>
> Peter Schrott 于2021年10月18日周一 下午8:40写道:
>
>> Hi there,
>>
>> I have a Kafka topic where the schema of its value
Hi there,
I have a Kafka topic where the schema of its values is defined by the
"MyRecord" record in the following Avro IDL and registered to the Confluent
Schema Registry:
@namespace("org.example")
protocol MyProtocol {
record MyRecord {
string text;
}
}
The topic is consumed with a
33 matches
Mail list logo