[
https://issues.apache.org/jira/browse/FLINK-19156?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
]
Dawid Wysakowicz updated FLINK-19156:
-------------------------------------
Description:
The code that checks if we should migrate the transaction id is as follows:
{code}
@Deprecated
private static final
ListStateDescriptor<FlinkKafkaProducer.NextTransactionalIdHint>
NEXT_TRANSACTIONAL_ID_HINT_DESCRIPTOR =
new ListStateDescriptor<>("next-transactional-id-hint",
TypeInformation.of(NextTransactionalIdHint.class));
if
(context.getOperatorStateStore().getRegisteredStateNames().contains(NEXT_TRANSACTIONAL_ID_HINT_DESCRIPTOR))
{
migrateNextTransactionalIdHindState(context);
}
{code}
The condition in if statement is never met because it checks if a
{{Set<String>}} contains object of type {{ListStateDescriptor}}
was:
The code that checks if we should migrate the transaction id is as follows:
{code}
@Deprecated
private static final
ListStateDescriptor<FlinkKafkaProducer.NextTransactionalIdHint>
NEXT_TRANSACTIONAL_ID_HINT_DESCRIPTOR =
new ListStateDescriptor<>("next-transactional-id-hint",
TypeInformation.of(NextTransactionalIdHint.class));
if
(context.getOperatorStateStore().getRegisteredStateNames().contains(NEXT_TRANSACTIONAL_ID_HINT_DESCRIPTOR))
{
migrateNextTransactionalIdHindState(context);
}
{code}
The condition in if statement is never met because it checks if a
{{Set<String>}} contains object of type {{ListStateDescriptor}}
> Migration of transactionIdHint in Kafka is never applied
> --------------------------------------------------------
>
> Key: FLINK-19156
> URL: https://issues.apache.org/jira/browse/FLINK-19156
> Project: Flink
> Issue Type: Bug
> Components: Connectors / Kafka
> Affects Versions: 1.9.0
> Reporter: Dawid Wysakowicz
> Priority: Critical
>
> The code that checks if we should migrate the transaction id is as follows:
> {code}
> @Deprecated
> private static final
> ListStateDescriptor<FlinkKafkaProducer.NextTransactionalIdHint>
> NEXT_TRANSACTIONAL_ID_HINT_DESCRIPTOR =
> new ListStateDescriptor<>("next-transactional-id-hint",
> TypeInformation.of(NextTransactionalIdHint.class));
> if
> (context.getOperatorStateStore().getRegisteredStateNames().contains(NEXT_TRANSACTIONAL_ID_HINT_DESCRIPTOR))
> {
> migrateNextTransactionalIdHindState(context);
> }
> {code}
> The condition in if statement is never met because it checks if a
> {{Set<String>}} contains object of type {{ListStateDescriptor}}
--
This message was sent by Atlassian Jira
(v8.3.4#803005)