diff --git a/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/AbstractIgniteCdcStreamer.java b/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/AbstractIgniteCdcStreamer.java index f56a9954..0553dc68 100644 --- a/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/AbstractIgniteCdcStreamer.java +++ b/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/AbstractIgniteCdcStreamer.java @@ -17,9 +17,18 @@ package org.apache.ignite.cdc; +import java.io.IOException; +import java.nio.file.Files; +import java.nio.file.Path; +import java.util.HashSet; import java.util.Iterator; +import java.util.List; +import java.util.Optional; import java.util.Set; +import java.util.regex.Pattern; +import java.util.regex.PatternSyntaxException; import java.util.stream.Collectors; + import org.apache.ignite.IgniteCheckedException; import org.apache.ignite.IgniteException; import org.apache.ignite.IgniteLogger; @@ -35,6 +44,8 @@ import org.apache.ignite.metric.MetricRegistry; import org.apache.ignite.resources.LoggerResource; +import static java.nio.file.StandardCopyOption.ATOMIC_MOVE; +import static java.nio.file.StandardCopyOption.REPLACE_EXISTING; import static org.apache.ignite.cdc.kafka.IgniteToKafkaCdcStreamer.DFLT_IS_ONLY_PRIMARY; /** @@ -67,12 +78,33 @@ public abstract class AbstractIgniteCdcStreamer implements CdcConsumer { /** */ public static final String LAST_EVT_SENT_TIME_DESC = "Timestamp of last applied event to destination cluster"; + /** File with saved names of caches added by cache masks. */ + private static final String SAVED_CACHES_FILE = "caches"; + + /** Temporary file with saved names of caches added by cache masks. */ + private static final String SAVED_CACHES_TMP_FILE = "caches_tmp"; + + /** CDC directory path. */ + private Path cdcDir; + /** Handle only primary entry flag. */ private boolean onlyPrimary = DFLT_IS_ONLY_PRIMARY; /** Cache names. */ private Set caches; + /** Include regex templates for cache names. */ + private Set includeTemplates = new HashSet<>(); + + /** Compiled include regex patterns for cache names. */ + private Set includeFilters; + + /** Exclude regex templates for cache names. */ + private Set excludeTemplates = new HashSet<>(); + + /** Compiled exclude regex patterns for cache names. */ + private Set excludeFilters; + /** Cache IDs. */ protected Set cachesIds; @@ -99,14 +131,28 @@ public abstract class AbstractIgniteCdcStreamer implements CdcConsumer { protected IgniteLogger log; /** {@inheritDoc} */ - @Override public void start(MetricRegistry reg) { + @Override public void start(MetricRegistry reg, Path cdcDir) { A.notEmpty(caches, "caches"); + this.cdcDir = cdcDir; + cachesIds = caches.stream() .mapToInt(CU::cacheId) .boxed() .collect(Collectors.toSet()); + prepareRegexFilters(); + + try { + loadCaches().stream() + .filter(this::matchesFilters) + .map(CU::cacheId) + .forEach(cachesIds::add); + } + catch (IOException e) { + throw new IgniteException(e); + } + MetricRegistryImpl mreg = (MetricRegistryImpl)reg; this.evtsCnt = mreg.longMetric(EVTS_SENT_CNT, EVTS_SENT_CNT_DESC); @@ -144,15 +190,147 @@ public abstract class AbstractIgniteCdcStreamer implements CdcConsumer { /** {@inheritDoc} */ @Override public void onCacheChange(Iterator cacheEvents) { cacheEvents.forEachRemaining(e -> { - // Just skip. Handle of cache events not supported. + matchWithRegexTemplates(e.configuration().getName()); }); } + /** + * Finds match between cache name and user's regex templates. + * If match is found, adds this cache's id to id's list and saves cache name to file. + * + * @param cacheName Cache name. + */ + private void matchWithRegexTemplates(String cacheName) { + int cacheId = CU.cacheId(cacheName); + + if (!cachesIds.contains(cacheId) && matchesFilters(cacheName)) { + cachesIds.add(cacheId); + + try { + List caches = loadCaches(); + + caches.add(cacheName); + + save(caches); + } + catch (IOException e) { + throw new IgniteException(e); + } + + if (log.isInfoEnabled()) + log.info("Cache has been added to replication [cacheName=" + cacheName + "]"); + } + } + + /** + * Writes caches list to file + * + * @param caches Caches list. + */ + private void save(List caches) throws IOException { + if (cdcDir == null) { + throw new IgniteException("Can't write to '" + SAVED_CACHES_FILE + "' file. Cdc directory is null"); + } + Path savedCachesPath = cdcDir.resolve(SAVED_CACHES_FILE); + Path tmpSavedCachesPath = cdcDir.resolve(SAVED_CACHES_TMP_FILE); + + StringBuilder cacheList = new StringBuilder(); + + for (String cache : caches) { + cacheList.append(cache); + + cacheList.append('\n'); + } + + Files.write(tmpSavedCachesPath, cacheList.toString().getBytes()); + + Files.move(tmpSavedCachesPath, savedCachesPath, ATOMIC_MOVE, REPLACE_EXISTING); + } + + /** + * Loads saved caches from file. + * + * @return List of saved caches names. + */ + private List loadCaches() throws IOException { + if (cdcDir == null) { + throw new IgniteException("Can't load '" + SAVED_CACHES_FILE + "' file. Cdc directory is null"); + } + Path savedCachesPath = cdcDir.resolve(SAVED_CACHES_FILE); + + if (Files.notExists(savedCachesPath)) { + Files.createFile(savedCachesPath); + + if (log.isInfoEnabled()) + log.info("Cache list created: " + savedCachesPath); + } + + return Files.readAllLines(savedCachesPath); + } + + /** + * Compiles regex patterns from user templates. + * + * @throws PatternSyntaxException If the template's syntax is invalid + */ + private void prepareRegexFilters() { + includeFilters = includeTemplates.stream() + .map(Pattern::compile) + .collect(Collectors.toSet()); + + excludeFilters = excludeTemplates.stream() + .map(Pattern::compile) + .collect(Collectors.toSet()); + } + + /** + * Matches cache name with compiled regex patterns. + * + * @param cacheName Cache name. + * @return True if cache name match include patterns and don't match exclude patterns. + */ + private boolean matchesFilters(String cacheName) { + boolean matchesInclude = includeFilters.stream() + .anyMatch(pattern -> pattern.matcher(cacheName).matches()); + + boolean notMatchesExclude = excludeFilters.stream() + .noneMatch(pattern -> pattern.matcher(cacheName).matches()); + + return matchesInclude && notMatchesExclude; + } + /** {@inheritDoc} */ @Override public void onCacheDestroy(Iterator caches) { - caches.forEachRemaining(e -> { - // Just skip. Handle of cache events not supported. - }); + caches.forEachRemaining(this::deleteRegexpCacheIfPresent); + } + + /** + * Removes cache added by regexp from cache list, if this cache is present in file, to prevent file size overflow. + * + * @param cacheId Cache id. + */ + private void deleteRegexpCacheIfPresent(Integer cacheId) { + try { + List caches = loadCaches(); + + Optional cacheName = caches.stream() + .filter(name -> CU.cacheId(name) == cacheId) + .findAny(); + + if (cacheName.isPresent()) { + String name = cacheName.get(); + + caches.remove(name); + + save(caches); + + if (log.isInfoEnabled()) + log.info("Cache has been removed from replication [cacheName=" + name + ']'); + } + } + catch (IOException e) { + throw new IgniteException(e); + } } /** {@inheritDoc} */ @@ -238,6 +416,30 @@ public AbstractIgniteCdcStreamer setCaches(Set caches) { return this; } + /** + * Sets include regex patterns that participate in CDC. + * + * @param includeTemplates Include regex templates + * @return {@code this} for chaining. + */ + public AbstractIgniteCdcStreamer setIncludeTemplates(Set includeTemplates) { + this.includeTemplates = includeTemplates; + + return this; + } + + /** + * Sets exclude regex patterns that participate in CDC. + * + * @param excludeTemplates Exclude regex templates + * @return {@code this} for chaining. + */ + public AbstractIgniteCdcStreamer setExcludeTemplates(Set excludeTemplates) { + this.excludeTemplates = excludeTemplates; + + return this; + } + /** * Sets maximum batch size that will be applied to destination cluster. * diff --git a/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/IgniteToIgniteCdcStreamer.java b/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/IgniteToIgniteCdcStreamer.java index 618c61d7..902b8ca6 100644 --- a/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/IgniteToIgniteCdcStreamer.java +++ b/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/IgniteToIgniteCdcStreamer.java @@ -17,6 +17,8 @@ package org.apache.ignite.cdc; +import java.nio.file.Path; + import org.apache.ignite.IgniteException; import org.apache.ignite.Ignition; import org.apache.ignite.cdc.conflictresolve.CacheVersionConflictResolverImpl; @@ -61,8 +63,8 @@ public class IgniteToIgniteCdcStreamer extends AbstractIgniteCdcStreamer impleme private volatile boolean alive = true; /** {@inheritDoc} */ - @Override public void start(MetricRegistry mreg) { - super.start(mreg); + @Override public void start(MetricRegistry mreg, Path cdcDir) { + super.start(mreg, cdcDir); if (log.isInfoEnabled()) log.info("Ignite To Ignite Streamer [cacheIds=" + cachesIds + ']'); diff --git a/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/conflictresolve/CacheVersionConflictResolverPluginProvider.java b/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/conflictresolve/CacheVersionConflictResolverPluginProvider.java index 0083f136..f283b6c2 100644 --- a/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/conflictresolve/CacheVersionConflictResolverPluginProvider.java +++ b/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/conflictresolve/CacheVersionConflictResolverPluginProvider.java @@ -18,8 +18,11 @@ package org.apache.ignite.cdc.conflictresolve; import java.io.Serializable; +import java.util.HashSet; import java.util.Set; import java.util.UUID; +import java.util.regex.Pattern; + import org.apache.ignite.IgniteLogger; import org.apache.ignite.cluster.ClusterNode; import org.apache.ignite.internal.IgniteEx; @@ -65,6 +68,12 @@ public class CacheVersionConflictResolverPluginProvider includeTemplates = new HashSet<>(); + + /** Exclude regex templates for cache names. */ + private Set excludeTemplates = new HashSet<>(); + /** Log. */ private IgniteLogger log; @@ -98,7 +107,7 @@ public CacheVersionConflictResolverPluginProvider() { @Override public CachePluginProvider createCacheProvider(CachePluginContext ctx) { String cacheName = ctx.igniteCacheConfiguration().getName(); - if (caches.contains(cacheName)) { + if (caches.contains(cacheName) || matchesFilters(cacheName)) { log.info("ConflictResolver provider set for cache [cacheName=" + cacheName + ']'); return provider; @@ -144,6 +153,16 @@ public void setConflictResolver(CacheVersionConflictResolver resolver) { this.resolver = resolver; } + /** @param includeTemplates Include regex templates */ + public void setIncludeTemplates(Set includeTemplates) { + this.includeTemplates = includeTemplates; + } + + /** @param excludeTemplates Exclude regex templates */ + public void setExcludeTemplates(Set excludeTemplates) { + this.excludeTemplates = excludeTemplates; + } + /** {@inheritDoc} */ @Override public void start(PluginContext ctx) { ((IgniteEx)ctx.grid()).context().cache().context().versions().dataCenterId(clusterId); @@ -178,4 +197,21 @@ public void setConflictResolver(CacheVersionConflictResolver resolver) { @Nullable @Override public T createComponent(PluginContext ctx, Class cls) { return null; } + + /** + * Match cache name with regex patterns. + * + * @param cacheName Cache name. + */ + private boolean matchesFilters(String cacheName) { + boolean matchesInclude = includeTemplates.stream() + .map(Pattern::compile) + .anyMatch(pattern -> pattern.matcher(cacheName).matches()); + + boolean notMatchesExclude = excludeTemplates.stream() + .map(Pattern::compile) + .noneMatch(pattern -> pattern.matcher(cacheName).matches()); + + return matchesInclude && notMatchesExclude; + } } diff --git a/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/kafka/AbstractKafkaToIgniteCdcStreamer.java b/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/kafka/AbstractKafkaToIgniteCdcStreamer.java index 5332f1bd..40102a50 100644 --- a/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/kafka/AbstractKafkaToIgniteCdcStreamer.java +++ b/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/kafka/AbstractKafkaToIgniteCdcStreamer.java @@ -19,11 +19,11 @@ import java.util.ArrayList; import java.util.Collection; +import java.util.HashSet; import java.util.List; import java.util.Properties; import java.util.Set; import java.util.concurrent.atomic.AtomicBoolean; -import java.util.stream.Collectors; import org.apache.ignite.IgniteException; import org.apache.ignite.IgniteLogger; import org.apache.ignite.cdc.AbstractCdcEventsApplier; @@ -152,13 +152,13 @@ public AbstractKafkaToIgniteCdcStreamer(Properties kafkaProps, KafkaToIgniteCdcS protected void runAppliers() { AtomicBoolean stopped = new AtomicBoolean(); - Set caches = null; + Set caches = new HashSet<>(); if (!F.isEmpty(streamerCfg.getCaches())) { checkCaches(streamerCfg.getCaches()); - caches = streamerCfg.getCaches().stream() - .map(CU::cacheId).collect(Collectors.toSet()); + streamerCfg.getCaches().stream() + .map(CU::cacheId).forEach(caches::add); } KafkaToIgniteMetadataUpdater metaUpdr = new KafkaToIgniteMetadataUpdater( @@ -181,7 +181,8 @@ protected void runAppliers() { caches, metaUpdr, stopped, - metrics + metrics, + this ); addAndStart("applier-thread-" + cntr++, applier); @@ -252,6 +253,13 @@ private void addAndStart(String threadName, /** Checks that configured caches exist in a destination cluster. */ protected abstract void checkCaches(Collection caches); + /** + * Get cache names from client. + * + * @return Cache names. + * */ + protected abstract Collection getCaches(); + /** */ private void ackAsciiLogo(IgniteLogger log) { String ver = "ver. " + ACK_VER_STR; diff --git a/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/kafka/IgniteToKafkaCdcStreamer.java b/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/kafka/IgniteToKafkaCdcStreamer.java index db1ffa49..1ba3113a 100644 --- a/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/kafka/IgniteToKafkaCdcStreamer.java +++ b/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/kafka/IgniteToKafkaCdcStreamer.java @@ -17,10 +17,15 @@ package org.apache.ignite.cdc.kafka; +import java.io.IOException; +import java.nio.file.Files; +import java.nio.file.Path; import java.util.ArrayList; import java.util.Collection; +import java.util.HashSet; import java.util.Iterator; import java.util.List; +import java.util.Optional; import java.util.Properties; import java.util.Set; import java.util.concurrent.ExecutionException; @@ -28,8 +33,12 @@ import java.util.concurrent.TimeUnit; import java.util.concurrent.TimeoutException; import java.util.function.Function; +import java.util.regex.Pattern; +import java.util.regex.PatternSyntaxException; import java.util.stream.Collectors; import java.util.stream.IntStream; + +import org.apache.ignite.IgniteException; import org.apache.ignite.IgniteLogger; import org.apache.ignite.binary.BinaryType; import org.apache.ignite.cdc.CdcCacheEvent; @@ -55,6 +64,8 @@ import org.apache.kafka.common.serialization.ByteArraySerializer; import org.apache.kafka.common.serialization.IntegerSerializer; +import static java.nio.file.StandardCopyOption.ATOMIC_MOVE; +import static java.nio.file.StandardCopyOption.REPLACE_EXISTING; import static org.apache.ignite.cdc.kafka.KafkaToIgniteCdcStreamerConfiguration.DFLT_KAFKA_REQ_TIMEOUT; import static org.apache.ignite.cdc.kafka.KafkaToIgniteCdcStreamerConfiguration.DFLT_MAX_BATCH_SIZE; import static org.apache.kafka.clients.producer.ProducerConfig.KEY_SERIALIZER_CLASS_CONFIG; @@ -149,6 +160,27 @@ public class IgniteToKafkaCdcStreamer implements CdcConsumer { /** Cache names. */ private Collection caches; + /** File with saved names of caches added by cache masks. */ + private static final String SAVED_CACHES_FILE = "caches"; + + /** Temporary file with saved names of caches added by cache masks. */ + private static final String SAVED_CACHES_TMP_FILE = "caches_tmp"; + + /** CDC directory path. */ + private Path cdcDir; + + /** Include regex templates for cache names. */ + private Set includeTemplates = new HashSet<>(); + + /** Compiled include regex patterns for cache names. */ + private Set includeFilters; + + /** Exclude regex templates for cache names. */ + private Set excludeTemplates = new HashSet<>(); + + /** Compiled exclude regex patterns for cache names. */ + private Set excludeFilters; + /** Max batch size. */ private int maxBatchSz = DFLT_MAX_BATCH_SIZE; @@ -248,15 +280,42 @@ public class IgniteToKafkaCdcStreamer implements CdcConsumer { /** {@inheritDoc} */ @Override public void onCacheChange(Iterator cacheEvents) { cacheEvents.forEachRemaining(e -> { - // Just skip. Handle of cache events not supported. + matchWithRegexTemplates(e.configuration().getName()); }); } /** {@inheritDoc} */ @Override public void onCacheDestroy(Iterator caches) { - caches.forEachRemaining(e -> { - // Just skip. Handle of cache events not supported. - }); + caches.forEachRemaining(this::deleteRegexpCacheIfPresent); + } + + /** + * Removes cache added by regexp from cache list, if this cache is present in file, to prevent file size overflow. + * + * @param cacheId Cache id. + */ + private void deleteRegexpCacheIfPresent(Integer cacheId) { + try { + List caches = loadCaches(); + + Optional cacheName = caches.stream() + .filter(name -> CU.cacheId(name) == cacheId) + .findAny(); + + if (cacheName.isPresent()) { + String name = cacheName.get(); + + caches.remove(name); + + save(caches); + + if (log.isInfoEnabled()) + log.info("Cache has been removed from replication [cacheName=" + name + ']'); + } + } + catch (IOException e) { + throw new IgniteException(e); + } } /** Send marker(meta need to be updated) record to each partition of events topic. */ @@ -320,7 +379,7 @@ private void sendOneBatch( } /** {@inheritDoc} */ - @Override public void start(MetricRegistry reg) { + @Override public void start(MetricRegistry reg, Path cdcDir) { A.notNull(kafkaProps, "Kafka properties"); A.notNull(evtTopic, "Kafka topic"); A.notNull(metadataTopic, "Kafka metadata topic"); @@ -331,10 +390,24 @@ private void sendOneBatch( kafkaProps.setProperty(KEY_SERIALIZER_CLASS_CONFIG, IntegerSerializer.class.getName()); kafkaProps.setProperty(VALUE_SERIALIZER_CLASS_CONFIG, ByteArraySerializer.class.getName()); + this.cdcDir = cdcDir; + cachesIds = caches.stream() .map(CU::cacheId) .collect(Collectors.toSet()); + prepareRegexFilters(); + + try { + loadCaches().stream() + .filter(this::matchesFilters) + .map(CU::cacheId) + .forEach(cachesIds::add); + } + catch (IOException e) { + throw new IgniteException(e); + } + try { producer = new KafkaProducer<>(kafkaProps); @@ -380,6 +453,111 @@ public IgniteToKafkaCdcStreamer setOnlyPrimary(boolean onlyPrimary) { return this; } + /** + * Compiles regex patterns from user templates. + * + * @throws PatternSyntaxException If the template's syntax is invalid + */ + private void prepareRegexFilters() { + includeFilters = includeTemplates.stream() + .map(Pattern::compile) + .collect(Collectors.toSet()); + + excludeFilters = excludeTemplates.stream() + .map(Pattern::compile) + .collect(Collectors.toSet()); + } + + /** + * Loads saved caches from file. + * + * @return List of saved caches names. + */ + private List loadCaches() throws IOException { + if (cdcDir == null) { + throw new IgniteException("Can't load '" + SAVED_CACHES_FILE + "' file. Cdc directory is null"); + } + Path savedCachesPath = cdcDir.resolve(SAVED_CACHES_FILE); + + if (Files.notExists(savedCachesPath)) { + Files.createFile(savedCachesPath); + + if (log.isInfoEnabled()) + log.info("Cache list created: " + savedCachesPath); + } + + return Files.readAllLines(savedCachesPath); + } + + /** + * Matches cache name with compiled regex patterns. + * + * @param cacheName Cache name. + * @return True if cache name match include patterns and don't match exclude patterns. + */ + private boolean matchesFilters(String cacheName) { + boolean matchesInclude = includeFilters.stream() + .anyMatch(pattern -> pattern.matcher(cacheName).matches()); + + boolean notMatchesExclude = excludeFilters.stream() + .noneMatch(pattern -> pattern.matcher(cacheName).matches()); + + return matchesInclude && notMatchesExclude; + } + + /** + * Finds match between cache name and user's regex templates. + * If match is found, adds this cache's id to id's list and saves cache name to file. + * + * @param cacheName Cache name. + */ + private void matchWithRegexTemplates(String cacheName) { + int cacheId = CU.cacheId(cacheName); + + if (!cachesIds.contains(cacheId) && matchesFilters(cacheName)) { + cachesIds.add(cacheId); + + try { + List caches = loadCaches(); + + caches.add(cacheName); + + save(caches); + } + catch (IOException e) { + throw new IgniteException(e); + } + + if (log.isInfoEnabled()) + log.info("Cache has been added to replication [cacheName=" + cacheName + "]"); + } + } + + /** + * Writes caches list to file + * + * @param caches Caches list. + */ + private void save(List caches) throws IOException { + if (cdcDir == null) { + throw new IgniteException("Can't write to '" + SAVED_CACHES_FILE + "' file. Cdc directory is null"); + } + Path savedCachesPath = cdcDir.resolve(SAVED_CACHES_FILE); + Path tmpSavedCachesPath = cdcDir.resolve(SAVED_CACHES_TMP_FILE); + + StringBuilder cacheList = new StringBuilder(); + + for (String cache : caches) { + cacheList.append(cache); + + cacheList.append('\n'); + } + + Files.write(tmpSavedCachesPath, cacheList.toString().getBytes()); + + Files.move(tmpSavedCachesPath, savedCachesPath, ATOMIC_MOVE, REPLACE_EXISTING); + } + /** * Sets topic that is used to send data to Kafka. * @@ -428,6 +606,30 @@ public IgniteToKafkaCdcStreamer setCaches(Collection caches) { return this; } + /** + * Sets include regex patterns that participate in CDC. + * + * @param includeTemplates Include regex templates. + * @return {@code this} for chaining. + */ + public IgniteToKafkaCdcStreamer setIncludeTemplates(Set includeTemplates) { + this.includeTemplates = includeTemplates; + + return this; + } + + /** + * Sets exclude regex patterns that participate in CDC. + * + * @param excludeTemplates Exclude regex templates + * @return {@code this} for chaining. + */ + public IgniteToKafkaCdcStreamer setExcludeTemplates(Set excludeTemplates) { + this.excludeTemplates = excludeTemplates; + + return this; + } + /** * Sets maximum batch size. * diff --git a/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/kafka/KafkaToIgniteCdcStreamer.java b/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/kafka/KafkaToIgniteCdcStreamer.java index 3386a4c0..75f4f205 100644 --- a/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/kafka/KafkaToIgniteCdcStreamer.java +++ b/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/kafka/KafkaToIgniteCdcStreamer.java @@ -127,4 +127,9 @@ public KafkaToIgniteCdcStreamer( @Override protected void checkCaches(Collection caches) { caches.forEach(name -> Objects.requireNonNull(ign.cache(name), name + " not exists!")); } + + /** {@inheritDoc} */ + @Override protected Collection getCaches() { + return ign.cacheNames(); + } } diff --git a/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/kafka/KafkaToIgniteCdcStreamerApplier.java b/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/kafka/KafkaToIgniteCdcStreamerApplier.java index c28e18ba..4b67189c 100644 --- a/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/kafka/KafkaToIgniteCdcStreamerApplier.java +++ b/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/kafka/KafkaToIgniteCdcStreamerApplier.java @@ -27,11 +27,14 @@ import java.util.Iterator; import java.util.List; import java.util.Map; +import java.util.Optional; import java.util.Properties; import java.util.Set; import java.util.concurrent.atomic.AtomicBoolean; import java.util.concurrent.atomic.AtomicLong; import java.util.function.Supplier; +import java.util.regex.Pattern; + import org.apache.ignite.IgniteCheckedException; import org.apache.ignite.IgniteException; import org.apache.ignite.IgniteLogger; @@ -42,6 +45,7 @@ import org.apache.ignite.internal.processors.cache.version.CacheVersionConflictResolver; import org.apache.ignite.internal.processors.cache.version.GridCacheVersion; import org.apache.ignite.internal.util.typedef.F; +import org.apache.ignite.internal.util.typedef.internal.CU; import org.apache.ignite.internal.util.typedef.internal.S; import org.apache.kafka.clients.consumer.ConsumerRecord; import org.apache.kafka.clients.consumer.ConsumerRecords; @@ -104,6 +108,12 @@ class KafkaToIgniteCdcStreamerApplier implements Runnable, AutoCloseable { /** Caches ids to read. */ private final Set caches; + /** Include regex templates for cache names. */ + private final Set includeTemplates; + + /** Exclude regex templates for cache names. */ + private final Set excludeTemplates; + /** The maximum time to complete Kafka related requests, in milliseconds. */ private final long kafkaReqTimeout; @@ -128,6 +138,9 @@ class KafkaToIgniteCdcStreamerApplier implements Runnable, AutoCloseable { /** CDC kafka to ignite metrics */ private final KafkaToIgniteMetrics metrics; + /** Instance of KafkaToIgniteCdcStreamer */ + private final AbstractKafkaToIgniteCdcStreamer streamer; + /** * @param applierSupplier Cdc events applier supplier. * @param log Logger. @@ -139,6 +152,7 @@ class KafkaToIgniteCdcStreamerApplier implements Runnable, AutoCloseable { * @param metaUpdr Metadata updater. * @param stopped Stopped flag. * @param metrics CDC Kafka to Ignite metrics. + * @param streamer Instance of KafkaToIgniteCdcStreamer */ public KafkaToIgniteCdcStreamerApplier( Supplier applierSupplier, @@ -150,7 +164,8 @@ public KafkaToIgniteCdcStreamerApplier( Set caches, KafkaToIgniteMetadataUpdater metaUpdr, AtomicBoolean stopped, - KafkaToIgniteMetrics metrics + KafkaToIgniteMetrics metrics, + AbstractKafkaToIgniteCdcStreamer streamer ) { this.applierSupplier = applierSupplier; this.kafkaProps = kafkaProps; @@ -164,6 +179,9 @@ public KafkaToIgniteCdcStreamerApplier( this.stopped = stopped; this.log = log.getLogger(KafkaToIgniteCdcStreamerApplier.class); this.metrics = metrics; + this.streamer = streamer; + this.includeTemplates = streamerCfg.getIncludeTemplates(); + this.excludeTemplates = streamerCfg.getExcludeTemplates(); } /** {@inheritDoc} */ @@ -260,7 +278,44 @@ private boolean filterAndPossiblyUpdateMetadata(ConsumerRecord metrics.incrementReceivedEvents(); - return F.isEmpty(caches) || caches.contains(rec.key()); + return F.isEmpty(caches) || caches.contains(rec.key()) || matchesRegexTemplates(rec.key()); + } + + /** + * Gets caches names from CDC client and finds match + * between cache id and user's regex templates. + * + * @param key Cache id. + * @return True if match is found. + */ + private boolean matchesRegexTemplates(Integer key) { + Optional cache = streamer.getCaches().stream() + .filter(name -> CU.cacheId(name) == key) + .findAny(); + + Optional matchedCache = cache.filter(this::matchesFilters); + + matchedCache.ifPresent(c -> caches.add(CU.cacheId(c))); + + return matchedCache.isPresent(); + } + + /** + * Matches cache name with compiled regex patterns. + * + * @param cacheName Cache name. + * @return True if cache name match include patterns and don't match exclude patterns. + */ + private boolean matchesFilters(String cacheName) { + boolean matchesInclude = includeTemplates.stream() + .map(Pattern::compile) + .anyMatch(pattern -> pattern.matcher(cacheName).matches()); + + boolean notMatchesExclude = excludeTemplates.stream() + .map(Pattern::compile) + .noneMatch(pattern -> pattern.matcher(cacheName).matches()); + + return matchesInclude && notMatchesExclude; } /** diff --git a/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/kafka/KafkaToIgniteCdcStreamerConfiguration.java b/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/kafka/KafkaToIgniteCdcStreamerConfiguration.java index 07b97e2b..9876037e 100644 --- a/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/kafka/KafkaToIgniteCdcStreamerConfiguration.java +++ b/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/kafka/KafkaToIgniteCdcStreamerConfiguration.java @@ -18,7 +18,10 @@ package org.apache.ignite.cdc.kafka; import java.util.Collection; +import java.util.HashSet; import java.util.Map; +import java.util.Set; + import org.apache.ignite.cdc.CdcConfiguration; import org.apache.ignite.internal.processors.cache.IgniteInternalCache; import org.apache.ignite.lang.IgniteExperimental; @@ -85,6 +88,12 @@ public class KafkaToIgniteCdcStreamerConfiguration { */ private Collection caches; + /** Include regex templates for cache names. */ + private Set includeTemplates = new HashSet<>(); + + /** Exclude regex templates for cache names. */ + private Set excludeTemplates = new HashSet<>(); + /** Metric exporter SPI. */ private MetricExporterSpi[] metricExporterSpi; @@ -175,6 +184,34 @@ public void setCaches(Collection caches) { this.caches = caches; } + /** + * @return Include regex templates + */ + public Set getIncludeTemplates() { + return includeTemplates; + } + + /** + * @param includeTemplates Include regex templates + */ + public void setIncludeTemplates(Set includeTemplates) { + this.includeTemplates = includeTemplates; + } + + /** + * @return Exclude regex templates + */ + public Set getExcludeTemplates() { + return excludeTemplates; + } + + /** + * @param excludeTemplates Exclude regex templates + */ + public void setExcludeTemplates(Set excludeTemplates) { + this.excludeTemplates = excludeTemplates; + } + /** @return The maximum time to complete Kafka related requests, in milliseconds. */ public long getKafkaRequestTimeout() { return kafkaReqTimeout; diff --git a/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/kafka/KafkaToIgniteClientCdcStreamer.java b/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/kafka/KafkaToIgniteClientCdcStreamer.java index 6d05aab7..5c85558e 100644 --- a/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/kafka/KafkaToIgniteClientCdcStreamer.java +++ b/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/kafka/KafkaToIgniteClientCdcStreamer.java @@ -127,4 +127,9 @@ public KafkaToIgniteClientCdcStreamer( caches.forEach(name -> A.ensure(clusterCaches.contains(name), name + " not exists!")); } + + /** {@inheritDoc} */ + @Override protected Collection getCaches() { + return client.cacheNames(); + } } diff --git a/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/thin/IgniteToIgniteClientCdcStreamer.java b/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/thin/IgniteToIgniteClientCdcStreamer.java index bc7af745..607ca19c 100644 --- a/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/thin/IgniteToIgniteClientCdcStreamer.java +++ b/modules/cdc-ext/src/main/java/org/apache/ignite/cdc/thin/IgniteToIgniteClientCdcStreamer.java @@ -17,6 +17,8 @@ package org.apache.ignite.cdc.thin; +import java.nio.file.Path; + import org.apache.ignite.Ignition; import org.apache.ignite.cdc.AbstractIgniteCdcStreamer; import org.apache.ignite.cdc.conflictresolve.CacheVersionConflictResolverImpl; @@ -66,8 +68,8 @@ public class IgniteToIgniteClientCdcStreamer extends AbstractIgniteCdcStreamer { private long aliveCheckTimeout = DFLT_ALIVE_CHECK_TIMEOUT; /** {@inheritDoc} */ - @Override public void start(MetricRegistry mreg) { - super.start(mreg); + @Override public void start(MetricRegistry mreg, Path cdcDir) { + super.start(mreg, cdcDir); if (log.isInfoEnabled()) log.info("Ignite To Ignite Client Streamer [cacheIds=" + cachesIds + ']'); diff --git a/modules/cdc-ext/src/test/java/org/apache/ignite/cdc/AbstractReplicationTest.java b/modules/cdc-ext/src/test/java/org/apache/ignite/cdc/AbstractReplicationTest.java index 9a5e18c7..c79657f7 100644 --- a/modules/cdc-ext/src/test/java/org/apache/ignite/cdc/AbstractReplicationTest.java +++ b/modules/cdc-ext/src/test/java/org/apache/ignite/cdc/AbstractReplicationTest.java @@ -25,6 +25,7 @@ import java.util.EnumSet; import java.util.HashSet; import java.util.List; +import java.util.Set; import java.util.concurrent.ThreadLocalRandom; import java.util.concurrent.TimeUnit; import java.util.function.BiConsumer; @@ -146,6 +147,18 @@ public static Collection parameters() { /** */ public static final String IGNORED_CACHE = "ignored-cache"; + /** */ + public static final String REGEX_INCLUDE_TEMPLATE_CACHE = "cdc_on_cache"; + + /** */ + public static final String REGEX_EXCLUDE_TEMPLATE_CACHE = "cdc_on_excluded_cache"; + + /** */ + public static final String REGEX_INCLUDE_PATTERN = "cdc_on.*"; + + /** */ + public static final String REGEX_EXCLUDE_PATTERN = "cdc_on_excluded.*"; + /** */ public static final byte SRC_CLUSTER_ID = 1; @@ -200,6 +213,8 @@ private enum WaitDataMode { cfgPlugin1.setClusterId(clusterId); cfgPlugin1.setCaches(new HashSet<>(Arrays.asList(ACTIVE_PASSIVE_CACHE, ACTIVE_ACTIVE_CACHE))); + cfgPlugin1.setIncludeTemplates(new HashSet<>(Arrays.asList(REGEX_INCLUDE_PATTERN))); + cfgPlugin1.setExcludeTemplates(new HashSet<>(Arrays.asList(REGEX_EXCLUDE_PATTERN))); cfgPlugin1.setConflictResolveField("reqId"); cfg.setPluginProviders(cfgPlugin1); @@ -562,6 +577,98 @@ public void testWithExpiryPolicy() throws Exception { } } + /** Check that caches matching regex filters in config, are added to CDC after its creation. + * Active/Active mode means changes made in both clusters. */ + @Test + public void testActiveActiveReplicationWithRegexFilters() throws Exception { + Set includeTemplates = new HashSet<>(Arrays.asList(REGEX_INCLUDE_PATTERN)); + Set excludeTemplates = new HashSet<>(Arrays.asList(REGEX_EXCLUDE_PATTERN)); + + createCache(srcCluster[0], ACTIVE_ACTIVE_CACHE); + createCache(destCluster[0], ACTIVE_ACTIVE_CACHE); + + IgniteCache srcCache = createCache(srcCluster[0], REGEX_INCLUDE_TEMPLATE_CACHE); + IgniteCache destCache = createCache(destCluster[0], REGEX_INCLUDE_TEMPLATE_CACHE); + + // Even keys goes to src cluster. + runAsync(generateData(REGEX_INCLUDE_TEMPLATE_CACHE, srcCluster[srcCluster.length - 1], + IntStream.range(0, KEYS_CNT).filter(i -> i % 2 == 0))); + + // Odd keys goes to dest cluster. + runAsync(generateData(REGEX_INCLUDE_TEMPLATE_CACHE, destCluster[destCluster.length - 1], + IntStream.range(0, KEYS_CNT).filter(i -> i % 2 != 0))); + + //Start CDC with only 'active-active-cache' in 'caches' property of CDC config + List> futs = startActiveActiveCdcWithFilters(includeTemplates, excludeTemplates); + + try { + waitForSameData(srcCache, destCache, KEYS_CNT, WaitDataMode.EXISTS, futs); + + runAsync(() -> IntStream.range(0, KEYS_CNT).filter(j -> j % 2 == 0).forEach(srcCache::remove)); + runAsync(() -> IntStream.range(0, KEYS_CNT).filter(j -> j % 2 != 0).forEach(destCache::remove)); + + waitForSameData(srcCache, destCache, KEYS_CNT, WaitDataMode.REMOVED, futs); + + //Shouldn't add to the replication, otherwise CDC will throw an error + runAsync(generateData(REGEX_EXCLUDE_TEMPLATE_CACHE, srcCluster[srcCluster.length - 1], IntStream.range(0, KEYS_CNT))); + + assertFalse(destCluster[0].cacheNames().contains(REGEX_EXCLUDE_TEMPLATE_CACHE)); + } + finally { + for (IgniteInternalFuture fut : futs) + fut.cancel(); + } + } + + /** Check that caches matching regex filters in config, are added to CDC after its creation. + * Active/Passive mode means changes made only in one cluster. */ + @Test + public void testActivePassiveReplicationWithRegexFilters() throws Exception { + Set includeTemplates = new HashSet<>(Arrays.asList(REGEX_INCLUDE_PATTERN)); + Set excludeTemplates = new HashSet<>(Arrays.asList(REGEX_EXCLUDE_PATTERN)); + + //Start CDC with only 'active-active-cache' in 'caches' property of CDC config + List> futs = startActivePassiveCdcWithFilters(ACTIVE_PASSIVE_CACHE, + includeTemplates, excludeTemplates); + + try { + createCache(destCluster[0], ACTIVE_PASSIVE_CACHE); + + IgniteCache destCache = createCache(destCluster[0], REGEX_INCLUDE_TEMPLATE_CACHE); + + // Updates for "ignored-cache" should be ignored because of CDC consume configuration. + runAsync(generateData(IGNORED_CACHE, srcCluster[srcCluster.length - 1], IntStream.range(0, KEYS_CNT))); + runAsync(generateData(REGEX_INCLUDE_TEMPLATE_CACHE, srcCluster[srcCluster.length - 1], IntStream.range(0, KEYS_CNT))); + + IgniteCache srcCache = + createCache(srcCluster[srcCluster.length - 1], REGEX_INCLUDE_TEMPLATE_CACHE); + + waitForSameData(srcCache, destCache, KEYS_CNT, WaitDataMode.EXISTS, futs); + + checkMetricsCount(KEYS_CNT); + checkMetrics(); + + IntStream.range(0, KEYS_CNT).forEach(srcCache::remove); + + waitForSameData(srcCache, destCache, KEYS_CNT, WaitDataMode.REMOVED, futs); + + checkMetrics(); + + assertFalse(destCluster[0].cacheNames().contains(IGNORED_CACHE)); + + checkMetricsCount(2 * KEYS_CNT); + + //Shouldn't add to the replication, otherwise CDC will throw an error + runAsync(generateData(REGEX_EXCLUDE_TEMPLATE_CACHE, srcCluster[srcCluster.length - 1], IntStream.range(0, KEYS_CNT))); + + assertFalse(destCluster[0].cacheNames().contains(REGEX_EXCLUDE_TEMPLATE_CACHE)); + } + finally { + for (IgniteInternalFuture fut : futs) + fut.cancel(); + } + } + /** */ public Runnable generateData(String cacheName, IgniteEx ign, IntStream keys) { return () -> { @@ -688,9 +795,18 @@ protected String[] hostAddresses(IgniteEx[] dest) { /** */ protected abstract List> startActivePassiveCdc(String cache); + /** */ + protected abstract List> startActivePassiveCdcWithFilters(String cache, + Set includeTemplates, + Set excludeTemplates); + /** */ protected abstract List> startActiveActiveCdc(); + /** */ + protected abstract List> startActiveActiveCdcWithFilters(Set includeTemplates, + Set excludeTemplates); + /** */ protected abstract void checkConsumerMetrics(Function longMetric); diff --git a/modules/cdc-ext/src/test/java/org/apache/ignite/cdc/CdcIgniteToIgniteReplicationTest.java b/modules/cdc-ext/src/test/java/org/apache/ignite/cdc/CdcIgniteToIgniteReplicationTest.java index b6d42e24..9c021645 100644 --- a/modules/cdc-ext/src/test/java/org/apache/ignite/cdc/CdcIgniteToIgniteReplicationTest.java +++ b/modules/cdc-ext/src/test/java/org/apache/ignite/cdc/CdcIgniteToIgniteReplicationTest.java @@ -20,6 +20,7 @@ import java.util.ArrayList; import java.util.Collections; import java.util.List; +import java.util.Set; import java.util.function.Function; import org.apache.ignite.Ignition; import org.apache.ignite.cdc.thin.IgniteToIgniteClientCdcStreamer; @@ -45,26 +46,40 @@ public class CdcIgniteToIgniteReplicationTest extends AbstractReplicationTest { /** {@inheritDoc} */ @Override protected List> startActivePassiveCdc(String cache) { + return startActivePassiveCdcWithFilters(cache, Collections.emptySet(), Collections.emptySet()); + } + + /** {@inheritDoc} */ + @Override protected List> startActivePassiveCdcWithFilters(String cache, + Set includeTemplates, + Set excludeTemplates) { List> futs = new ArrayList<>(); for (int i = 0; i < srcCluster.length; i++) - futs.add(igniteToIgnite(srcCluster[i].configuration(), destClusterCliCfg[i], destCluster, cache, "ignite-to-ignite-src-" + i)); + futs.add(igniteToIgnite(srcCluster[i].configuration(), destClusterCliCfg[i], destCluster, cache, + includeTemplates, excludeTemplates, "ignite-to-ignite-src-" + i)); return futs; } /** {@inheritDoc} */ @Override protected List> startActiveActiveCdc() { + return startActiveActiveCdcWithFilters(Collections.emptySet(), Collections.emptySet()); + } + + /** {@inheritDoc} */ + @Override protected List> startActiveActiveCdcWithFilters(Set includeTemplates, + Set excludeTemplates) { List> futs = new ArrayList<>(); for (int i = 0; i < srcCluster.length; i++) { - futs.add(igniteToIgnite( - srcCluster[i].configuration(), destClusterCliCfg[i], destCluster, ACTIVE_ACTIVE_CACHE, "ignite-to-ignite-src-" + i)); + futs.add(igniteToIgnite(srcCluster[i].configuration(), destClusterCliCfg[i], destCluster, + ACTIVE_ACTIVE_CACHE, includeTemplates, excludeTemplates, "ignite-to-ignite-src-" + i)); } for (int i = 0; i < destCluster.length; i++) { - futs.add(igniteToIgnite( - destCluster[i].configuration(), srcClusterCliCfg[i], srcCluster, ACTIVE_ACTIVE_CACHE, "ignite-to-ignite-dest-" + i)); + futs.add(igniteToIgnite(destCluster[i].configuration(), srcClusterCliCfg[i], srcCluster, + ACTIVE_ACTIVE_CACHE, includeTemplates, excludeTemplates, "ignite-to-ignite-dest-" + i)); } return futs; @@ -86,6 +101,8 @@ public class CdcIgniteToIgniteReplicationTest extends AbstractReplicationTest { * @param destCfg Ignite destination cluster configuration. * @param dest Ignite destination cluster. * @param cache Cache name to stream to kafka. + * @param includeTemplates Include regex templates for cache names. + * @param excludeTemplates Exclude regex templates for cache names. * @param threadName Thread to run CDC instance. * @return Future for Change Data Capture application. */ @@ -94,6 +111,8 @@ protected IgniteInternalFuture igniteToIgnite( IgniteConfiguration destCfg, IgniteEx[] dest, String cache, + Set includeTemplates, + Set excludeTemplates, @Nullable String threadName ) { return runAsync(() -> { @@ -115,6 +134,8 @@ protected IgniteInternalFuture igniteToIgnite( streamer.setMaxBatchSize(KEYS_CNT); streamer.setCaches(Collections.singleton(cache)); + streamer.setIncludeTemplates(includeTemplates); + streamer.setExcludeTemplates(excludeTemplates); cdcCfg.setConsumer(streamer); cdcCfg.setMetricExporterSpi(new JmxMetricExporterSpi()); diff --git a/modules/cdc-ext/src/test/java/org/apache/ignite/cdc/RegexFiltersTest.java b/modules/cdc-ext/src/test/java/org/apache/ignite/cdc/RegexFiltersTest.java new file mode 100644 index 00000000..e4007ab0 --- /dev/null +++ b/modules/cdc-ext/src/test/java/org/apache/ignite/cdc/RegexFiltersTest.java @@ -0,0 +1,233 @@ +package org.apache.ignite.cdc; + +import java.util.Arrays; +import java.util.Collections; +import java.util.HashSet; +import java.util.Set; +import java.util.stream.IntStream; + +import org.apache.ignite.IgniteCache; +import org.apache.ignite.cache.CacheAtomicityMode; +import org.apache.ignite.cache.CacheMode; +import org.apache.ignite.cdc.thin.IgniteToIgniteClientCdcStreamer; +import org.apache.ignite.cluster.ClusterState; +import org.apache.ignite.configuration.CacheConfiguration; +import org.apache.ignite.configuration.ClientConfiguration; +import org.apache.ignite.configuration.DataRegionConfiguration; +import org.apache.ignite.configuration.DataStorageConfiguration; +import org.apache.ignite.configuration.IgniteConfiguration; +import org.apache.ignite.internal.IgniteEx; +import org.apache.ignite.internal.IgniteInternalFuture; +import org.apache.ignite.internal.IgniteInterruptedCheckedException; +import org.apache.ignite.internal.cdc.CdcMain; +import org.apache.ignite.internal.processors.odbc.ClientListenerProcessor; +import org.apache.ignite.internal.util.typedef.F; +import org.apache.ignite.internal.util.typedef.X; +import org.apache.ignite.spi.discovery.tcp.TcpDiscoverySpi; +import org.apache.ignite.spi.discovery.tcp.ipfinder.vm.TcpDiscoveryVmIpFinder; +import org.apache.ignite.spi.metric.jmx.JmxMetricExporterSpi; +import org.apache.ignite.testframework.junits.common.GridCommonAbstractTest; +import org.junit.Test; + +import static org.apache.ignite.spi.communication.tcp.TcpCommunicationSpi.DFLT_PORT_RANGE; +import static org.apache.ignite.testframework.GridTestUtils.runAsync; +import static org.apache.ignite.testframework.GridTestUtils.waitForCondition; + +/** */ +public class RegexFiltersTest extends GridCommonAbstractTest { + + /** */ + private IgniteEx src; + + /** */ + private IgniteEx dest; + + /** */ + private int discoPort = TcpDiscoverySpi.DFLT_PORT; + + /** */ + private enum WaitDataMode { + /** */ + EXISTS, + + /** */ + REMOVED + } + + /** */ + private static final String TEST_CACHE = "test-cache"; + + /** */ + private static final String REGEX_MATCHING_CACHE = "regex-cache"; + + /** */ + private static final String REGEX_INCLUDE_PATTERN = "regex.*"; + + /** */ + private Set includeTemplates; + + /** */ + private static final int KEYS_CNT = 1000; + + /** {@inheritDoc} */ + @Override protected IgniteConfiguration getConfiguration(String igniteInstanceName) throws Exception { + TcpDiscoveryVmIpFinder finder = new TcpDiscoveryVmIpFinder() + .setAddresses(Collections.singleton("127.0.0.1:" + discoPort + ".." + (discoPort + DFLT_PORT_RANGE))); + + IgniteConfiguration cfg = super.getConfiguration(igniteInstanceName) + .setDiscoverySpi(new TcpDiscoverySpi() + .setLocalPort(discoPort) + .setIpFinder(finder)); + + cfg.setDataStorageConfiguration(new DataStorageConfiguration() + .setDefaultDataRegionConfiguration(new DataRegionConfiguration() + .setPersistenceEnabled(true) + .setCdcEnabled(true))); + + cfg.getDataStorageConfiguration() + .setWalForceArchiveTimeout(5_000); + + cfg.setConsistentId(igniteInstanceName); + + return cfg; + } + + /** + * + * @param srcCfg Ignite source node configuration. + * @param cache Cache name to stream to Ignite2Ignite. + * @param includeTemplates Include cache templates. + * @param excludeTemplates Exclude cache templates. + * @return Future for Change Data Capture application. + */ + private IgniteInternalFuture startCdc(IgniteConfiguration srcCfg, + String cache, + Set includeTemplates, + Set excludeTemplates) { + return runAsync(() -> { + CdcConfiguration cdcCfg = new CdcConfiguration(); + + AbstractIgniteCdcStreamer streamer = new IgniteToIgniteClientCdcStreamer() + .setDestinationClientConfiguration(new ClientConfiguration() + .setAddresses(F.first(dest.localNode().addresses()) + ":" + + dest.localNode().attribute(ClientListenerProcessor.CLIENT_LISTENER_PORT))); + + streamer.setMaxBatchSize(KEYS_CNT); + streamer.setCaches(Collections.singleton(cache)); + streamer.setIncludeTemplates(includeTemplates); + streamer.setExcludeTemplates(excludeTemplates); + + cdcCfg.setConsumer(streamer); + cdcCfg.setMetricExporterSpi(new JmxMetricExporterSpi()); + + CdcMain cdc = new CdcMain(srcCfg, null, cdcCfg); + + cdc.run(); + }); + } + + /** {@inheritDoc} */ + @Override protected void beforeTest() throws Exception { + cleanPersistenceDir(); + + src = startGrid(getConfiguration("source-cluster")); + + discoPort += DFLT_PORT_RANGE + 1; + + dest = startGrid(getConfiguration("dest-cluster")); + + includeTemplates = new HashSet<>(Arrays.asList(REGEX_INCLUDE_PATTERN)); + } + + /** {@inheritDoc} */ + @Override protected void afterTest() throws Exception { + stopAllGrids(); + + cleanPersistenceDir(); + } + + /** */ + public void waitForSameData( + IgniteCache src, + IgniteCache dest, + int keysCnt, + WaitDataMode mode, + IgniteInternalFuture fut + ) throws IgniteInterruptedCheckedException { + assertTrue(waitForCondition(() -> { + for (int i = 0; i < keysCnt; i++) { + if (mode == WaitDataMode.EXISTS) { + if (!src.containsKey(i) || !dest.containsKey(i)) + return checkFut(false, fut); + } + else if (mode == WaitDataMode.REMOVED) { + if (src.containsKey(i) || dest.containsKey(i)) + return checkFut(false, fut); + + continue; + } + else + throw new IllegalArgumentException(mode + " not supported."); + + Integer data = dest.get(i); + + if (!data.equals(src.get(i))) + return checkFut(false, fut); + } + + return checkFut(true, fut); + }, getTestTimeout())); + } + + /** */ + private boolean checkFut(boolean res, IgniteInternalFuture fut) { + assertFalse("Fut error: " + X.getFullStackTrace(fut.error()), fut.isDone()); + + return res; + } + + /** */ + public Runnable generateData(IgniteCache cache, IntStream keys) { + return () -> { + keys.forEach(i -> cache.put(i, i * 2)); + }; + } + + /** + * Test checks whether caches added by regex filters are saved to and read from file after CDC restart. + */ + @Test + public void testRegexFiltersOnCdcRestart() throws Exception { + + src.cluster().state(ClusterState.ACTIVE); + + dest.cluster().state(ClusterState.ACTIVE); + + //Start CDC only with 'test-cache' in config and cache masks (regex filters) + IgniteInternalFuture cdc = startCdc(src.configuration(), TEST_CACHE, includeTemplates, Collections.emptySet()); + + IgniteCache srcCache = src.getOrCreateCache(new CacheConfiguration() + .setName(REGEX_MATCHING_CACHE) + .setCacheMode(CacheMode.PARTITIONED) + .setAtomicityMode(CacheAtomicityMode.TRANSACTIONAL)); + + IgniteCache destCache = dest.getOrCreateCache(new CacheConfiguration() + .setName(REGEX_MATCHING_CACHE) + .setCacheMode(CacheMode.PARTITIONED) + .setAtomicityMode(CacheAtomicityMode.TRANSACTIONAL)); + + cdc.cancel(); + + //Restart CDC + IgniteInternalFuture cdc2 = startCdc(src.configuration(), TEST_CACHE, includeTemplates, Collections.emptySet()); + + try { + runAsync(generateData(srcCache, IntStream.range(0, KEYS_CNT))); + + waitForSameData(srcCache, destCache, KEYS_CNT, WaitDataMode.EXISTS, cdc2); + } + finally { + cdc2.cancel(); + } + } +} diff --git a/modules/cdc-ext/src/test/java/org/apache/ignite/cdc/kafka/CdcKafkaReplicationAppsTest.java b/modules/cdc-ext/src/test/java/org/apache/ignite/cdc/kafka/CdcKafkaReplicationAppsTest.java index 927a7b25..6b252584 100644 --- a/modules/cdc-ext/src/test/java/org/apache/ignite/cdc/kafka/CdcKafkaReplicationAppsTest.java +++ b/modules/cdc-ext/src/test/java/org/apache/ignite/cdc/kafka/CdcKafkaReplicationAppsTest.java @@ -25,6 +25,7 @@ import java.util.Arrays; import java.util.HashMap; import java.util.Map; +import java.util.Set; import java.util.stream.Collectors; import org.apache.ignite.configuration.IgniteConfiguration; import org.apache.ignite.internal.IgniteEx; @@ -113,6 +114,8 @@ public class CdcKafkaReplicationAppsTest extends CdcKafkaReplicationTest { String topic, String metadataTopic, String cache, + Set includeTemplates, + Set excludeTemplates, String threadName ) { Map params = new HashMap<>(); @@ -141,6 +144,8 @@ public class CdcKafkaReplicationAppsTest extends CdcKafkaReplicationTest { IgniteEx[] dest, int partFrom, int partTo, + Set includeTemplates, + Set excludeTemplates, String threadName ) { Map params = new HashMap<>(); diff --git a/modules/cdc-ext/src/test/java/org/apache/ignite/cdc/kafka/CdcKafkaReplicationTest.java b/modules/cdc-ext/src/test/java/org/apache/ignite/cdc/kafka/CdcKafkaReplicationTest.java index a56b2941..72e75ce8 100644 --- a/modules/cdc-ext/src/test/java/org/apache/ignite/cdc/kafka/CdcKafkaReplicationTest.java +++ b/modules/cdc-ext/src/test/java/org/apache/ignite/cdc/kafka/CdcKafkaReplicationTest.java @@ -21,6 +21,7 @@ import java.util.Collections; import java.util.List; import java.util.Properties; +import java.util.Set; import java.util.function.Function; import java.util.function.Supplier; import javax.management.DynamicMBean; @@ -93,6 +94,13 @@ public class CdcKafkaReplicationTest extends AbstractReplicationTest { /** {@inheritDoc} */ @Override protected List> startActivePassiveCdc(String cache) { + return startActivePassiveCdcWithFilters(cache, Collections.emptySet(), Collections.emptySet()); + } + + /** {@inheritDoc} */ + @Override protected List> startActivePassiveCdcWithFilters(String cache, + Set includeTemplates, + Set excludeTemplates) { try { KAFKA.createTopic(cache, DFLT_PARTS, 1); @@ -107,7 +115,8 @@ public class CdcKafkaReplicationTest extends AbstractReplicationTest { for (IgniteEx ex : srcCluster) { int idx = getTestIgniteInstanceIndex(ex.name()); - futs.add(igniteToKafka(ex.configuration(), cache, SRC_DEST_META_TOPIC, cache, "ignite-src-to-kafka-" + idx)); + futs.add(igniteToKafka(ex.configuration(), cache, SRC_DEST_META_TOPIC, cache, includeTemplates, + excludeTemplates, "ignite-src-to-kafka-" + idx)); } for (int i = 0; i < destCluster.length; i++) { @@ -119,7 +128,9 @@ public class CdcKafkaReplicationTest extends AbstractReplicationTest { destCluster, i * (DFLT_PARTS / 2), (i + 1) * (DFLT_PARTS / 2), - "kafka-to-ignite-dest-" + i + includeTemplates, + excludeTemplates, + "kafka-to-ignite-dest-" + i )); } @@ -128,20 +139,26 @@ public class CdcKafkaReplicationTest extends AbstractReplicationTest { /** {@inheritDoc} */ @Override protected List> startActiveActiveCdc() { + return startActiveActiveCdcWithFilters(Collections.emptySet(), Collections.emptySet()); + } + + /** {@inheritDoc} */ + @Override protected List> startActiveActiveCdcWithFilters(Set includeTemplates, + Set excludeTemplates) { List> futs = new ArrayList<>(); for (IgniteEx ex : srcCluster) { int idx = getTestIgniteInstanceIndex(ex.name()); - - futs.add(igniteToKafka( - ex.configuration(), SRC_DEST_TOPIC, SRC_DEST_META_TOPIC, ACTIVE_ACTIVE_CACHE, "ignite-src-to-kafka-" + idx)); + + futs.add(igniteToKafka(ex.configuration(), SRC_DEST_TOPIC, SRC_DEST_META_TOPIC, ACTIVE_ACTIVE_CACHE, + includeTemplates, excludeTemplates, "ignite-src-to-kafka-" + idx)); } for (IgniteEx ex : destCluster) { int idx = getTestIgniteInstanceIndex(ex.name()); - - futs.add(igniteToKafka( - ex.configuration(), DEST_SRC_TOPIC, DEST_SRC_META_TOPIC, ACTIVE_ACTIVE_CACHE, "ignite-dest-to-kafka-" + idx)); + + futs.add(igniteToKafka(ex.configuration(), DEST_SRC_TOPIC, DEST_SRC_META_TOPIC, ACTIVE_ACTIVE_CACHE, + includeTemplates, excludeTemplates, "ignite-dest-to-kafka-" + idx)); } futs.add(kafkaToIgnite( @@ -152,6 +169,8 @@ public class CdcKafkaReplicationTest extends AbstractReplicationTest { destCluster, 0, DFLT_PARTS, + includeTemplates, + excludeTemplates, "kafka-to-ignite-src" )); @@ -163,6 +182,8 @@ public class CdcKafkaReplicationTest extends AbstractReplicationTest { srcCluster, 0, DFLT_PARTS, + includeTemplates, + excludeTemplates, "kafka-to-ignite-dest" )); @@ -255,25 +276,31 @@ private void checkK2IMetrics(Function longMetric) { * @param topic Kafka topic name. * @param metadataTopic Metadata topic name. * @param cache Cache name to stream to kafka. + * @param includeTemplates Include regex templates for cache names. + * @param excludeTemplates Exclude regex templates for cache names. * @return Future for Change Data Capture application. */ protected IgniteInternalFuture igniteToKafka( - IgniteConfiguration igniteCfg, - String topic, - String metadataTopic, - String cache, - String threadName + IgniteConfiguration igniteCfg, + String topic, + String metadataTopic, + String cache, + Set includeTemplates, + Set excludeTemplates, + String threadName ) { return runAsync(() -> { IgniteToKafkaCdcStreamer cdcCnsmr = new IgniteToKafkaCdcStreamer() - .setTopic(topic) - .setMetadataTopic(metadataTopic) - .setKafkaPartitions(DFLT_PARTS) - .setCaches(Collections.singleton(cache)) - .setMaxBatchSize(KEYS_CNT) - .setOnlyPrimary(false) - .setKafkaProperties(kafkaProperties()) - .setKafkaRequestTimeout(DFLT_KAFKA_REQ_TIMEOUT); + .setTopic(topic) + .setMetadataTopic(metadataTopic) + .setKafkaPartitions(DFLT_PARTS) + .setCaches(Collections.singleton(cache)) + .setIncludeTemplates(includeTemplates) + .setExcludeTemplates(excludeTemplates) + .setMaxBatchSize(KEYS_CNT) + .setOnlyPrimary(false) + .setKafkaProperties(kafkaProperties()) + .setKafkaRequestTimeout(DFLT_KAFKA_REQ_TIMEOUT); CdcConfiguration cdcCfg = new CdcConfiguration(); @@ -292,6 +319,8 @@ protected IgniteInternalFuture igniteToKafka( * @param cacheName Cache name. * @param igniteCfg Ignite configuration. * @param dest Destination Ignite cluster. + * @param includeTemplates Include regex templates for cache names. + * @param excludeTemplates Exclude regex templates for cache names. * @return Future for runed {@link KafkaToIgniteCdcStreamer}. */ protected IgniteInternalFuture kafkaToIgnite( @@ -302,6 +331,8 @@ protected IgniteInternalFuture kafkaToIgnite( IgniteEx[] dest, int fromPart, int toPart, + Set includeTemplates, + Set excludeTemplates, String threadName ) { KafkaToIgniteCdcStreamerConfiguration cfg = new KafkaToIgniteCdcStreamerConfiguration(); @@ -311,6 +342,8 @@ protected IgniteInternalFuture kafkaToIgnite( cfg.setThreadCount((toPart - fromPart) / 2); cfg.setCaches(Collections.singletonList(cacheName)); + cfg.setIncludeTemplates(includeTemplates); + cfg.setExcludeTemplates(excludeTemplates); cfg.setTopic(topic); cfg.setMetadataTopic(metadataTopic); cfg.setKafkaRequestTimeout(DFLT_KAFKA_REQ_TIMEOUT); diff --git a/modules/cdc-ext/src/test/java/org/apache/ignite/cdc/kafka/KafkaToIgniteMetadataUpdaterTest.java b/modules/cdc-ext/src/test/java/org/apache/ignite/cdc/kafka/KafkaToIgniteMetadataUpdaterTest.java index 5dc764fe..e6892204 100644 --- a/modules/cdc-ext/src/test/java/org/apache/ignite/cdc/kafka/KafkaToIgniteMetadataUpdaterTest.java +++ b/modules/cdc-ext/src/test/java/org/apache/ignite/cdc/kafka/KafkaToIgniteMetadataUpdaterTest.java @@ -155,7 +155,7 @@ private IgniteToKafkaCdcStreamer igniteToKafkaCdcStreamer() { GridTestUtils.setFieldValue(streamer, "log", listeningLog.getLogger(IgniteToKafkaCdcStreamer.class)); - streamer.start(new MetricRegistryImpl("test", null, null, log)); + streamer.start(new MetricRegistryImpl("test", null, null, log), null); return streamer; }