-
Notifications
You must be signed in to change notification settings - Fork 16
Commit
This commit does not belong to any branch on this repository, and may belong to a fork outside of the repository.
[FLINK-21373] Add RabbitMQ SinkV2 Implementation, Port Flink version …
…to 1.19
- Loading branch information
1 parent
66e323a
commit cdb6c0d
Showing
29 changed files
with
3,135 additions
and
599 deletions.
There are no files selected for viewing
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
9 changes: 9 additions & 0 deletions
9
flink-connector-rabbitmq/archunit-violations/a6cee285-bdbf-4479-a652-8143c2bc1a69
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Original file line number | Diff line number | Diff line change |
---|---|---|
@@ -0,0 +1,9 @@ | ||
Method <org.apache.flink.streaming.connectors.rabbitmq.RMQSource$RMQCollectorImpl.setMessageIdentifiers(java.lang.String, long)> calls method <org.apache.flink.util.Preconditions.checkNotNull(java.lang.Object, java.lang.String)> in (RMQSource.java:415) | ||
Method <org.apache.flink.streaming.connectors.rabbitmq.RMQSource.close()> calls method <org.apache.flink.util.ExceptionUtils.firstOrSuppressed(java.lang.Throwable, java.lang.Throwable)> in (RMQSource.java:303) | ||
Method <org.apache.flink.streaming.connectors.rabbitmq.RMQSource.close()> calls method <org.apache.flink.util.IOUtils.closeAll([Ljava.lang.AutoCloseable;)> in (RMQSource.java:300) | ||
Method <org.apache.flink.streaming.connectors.rabbitmq.RMQSource.open(org.apache.flink.configuration.Configuration)> calls method <org.apache.flink.api.common.serialization.RuntimeContextInitializationContextAdapters.deserializationAdapter(org.apache.flink.api.common.functions.RuntimeContext, java.util.function.Function)> in (RMQSource.java:275) | ||
Method <org.apache.flink.streaming.connectors.rabbitmq.RMQSource.open(org.apache.flink.configuration.Configuration)> calls method <org.apache.flink.streaming.api.operators.StreamingRuntimeContext.isCheckpointingEnabled()> in (RMQSource.java:254) | ||
Method <org.apache.flink.streaming.connectors.rabbitmq.RMQSource.open(org.apache.flink.configuration.Configuration)> calls method <org.apache.flink.util.IOUtils.closeAllQuietly([Ljava.lang.AutoCloseable;)> in (RMQSource.java:266) | ||
Method <org.apache.flink.streaming.connectors.rabbitmq.RMQSource.open(org.apache.flink.configuration.Configuration)> checks instanceof <org.apache.flink.streaming.api.operators.StreamingRuntimeContext> in (RMQSource.java:253) | ||
Method <org.apache.flink.streaming.connectors.rabbitmq.RMQSource.setupConnection()> is annotated with <org.apache.flink.annotation.VisibleForTesting> in (RMQSource.java:0) | ||
Method <org.apache.flink.streaming.connectors.rabbitmq.RMQSource.setupQueue()> is annotated with <org.apache.flink.annotation.VisibleForTesting> in (RMQSource.java:0) |
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
13 changes: 13 additions & 0 deletions
13
...onnector-rabbitmq/src/main/java/org/apache/flink/connector/rabbitmq/common/Constants.java
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Original file line number | Diff line number | Diff line change |
---|---|---|
@@ -0,0 +1,13 @@ | ||
package org.apache.flink.connector.rabbitmq.common; | ||
|
||
import org.apache.flink.annotation.PublicEvolving; | ||
|
||
/** Constants for the RabbitMQ connector. */ | ||
@PublicEvolving | ||
public class Constants { | ||
public static final String DEFAULT_EXCHANGE = ""; | ||
|
||
public static final int DEFAULT_MAX_INFLIGHT = 100; | ||
|
||
public static final boolean DEFAULT_FAIL_ON_ERROR = false; | ||
} |
23 changes: 23 additions & 0 deletions
23
...main/java/org/apache/flink/connector/rabbitmq/common/DefaultRabbitMQMessageConverter.java
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Original file line number | Diff line number | Diff line change |
---|---|---|
@@ -0,0 +1,23 @@ | ||
package org.apache.flink.connector.rabbitmq.common; | ||
|
||
import org.apache.flink.annotation.PublicEvolving; | ||
|
||
import static org.apache.flink.connector.rabbitmq.common.Constants.DEFAULT_EXCHANGE; | ||
|
||
/** | ||
* Default implementation of {@link RabbitMQMessageConverter}. | ||
* | ||
* @param <T> type of the message to be converted | ||
*/ | ||
@PublicEvolving | ||
public class DefaultRabbitMQMessageConverter<T> implements RabbitMQMessageConverter<T> { | ||
@Override | ||
public RabbitMQMessage<T> toRabbitMQMessage(T value) { | ||
return RabbitMQMessage.<T>builder().setMessage(value).setExchange(DEFAULT_EXCHANGE).build(); | ||
} | ||
|
||
@Override | ||
public boolean supportsExchangeRouting() { | ||
return false; | ||
} | ||
} |
Oops, something went wrong.