diff --git a/modules/core/src/main/java/org/apache/ignite/internal/cdc/CdcConsumerEx.java b/modules/core/src/main/java/org/apache/ignite/internal/cdc/CdcConsumerEx.java new file mode 100644 index 0000000000000..03fb207b59119 --- /dev/null +++ b/modules/core/src/main/java/org/apache/ignite/internal/cdc/CdcConsumerEx.java @@ -0,0 +1,40 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one or more + * contributor license agreements. See the NOTICE file distributed with + * this work for additional information regarding copyright ownership. + * The ASF licenses this file to You under the Apache License, Version 2.0 + * (the "License"); you may not use this file except in compliance with + * the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package org.apache.ignite.internal.cdc; + +import java.util.Iterator; + +import org.apache.ignite.cdc.CdcCacheEvent; +import org.apache.ignite.cdc.CdcConsumer; +import org.apache.ignite.metric.MetricRegistry; + +/** + * Extended CdcConsumer interface which provides overloaded {@link CdcConsumerEx#start(MetricRegistry, Iterator)} method + * required for CDC regex filters. + */ +public interface CdcConsumerEx extends CdcConsumer { + /** + * Starts the consumer. + * @param mreg Metric registry for consumer specific metrics. + * @param cacheEvents The iterator contains previously handled {@link CdcCacheEvent}s that represent the actual + * caches at the time the consumer started. Note that changes which occurred while the application was down (creates, + * destroys, edits) are not included. Such changes will be delivered via the regular notifications after this method + * returns. + */ + void start(MetricRegistry mreg, Iterator cacheEvents); +} diff --git a/modules/core/src/main/java/org/apache/ignite/internal/cdc/CdcMain.java b/modules/core/src/main/java/org/apache/ignite/internal/cdc/CdcMain.java index d7f22658fdd7a..075ed10abc2c9 100644 --- a/modules/core/src/main/java/org/apache/ignite/internal/cdc/CdcMain.java +++ b/modules/core/src/main/java/org/apache/ignite/internal/cdc/CdcMain.java @@ -339,7 +339,27 @@ public void runX() throws Exception { committedSegmentOffset.value(walState.get1().fileOffset()); } - consumer.start(mreg, kctx.metric().registry(metricName("cdc", "consumer"))); + Iterator cacheEvts = GridLocalConfigManager + .readCachesData( + ft, + kctx.marshallerContext().jdkMarshaller(), + igniteCfg) + .entrySet().stream() + .map(data -> { + int cacheId = data.getValue().cacheId(); + long lastModified = data.getKey().lastModified(); + + Long lastModified0 = cachesState.get(cacheId); + + if (lastModified0 != null && lastModified0 == lastModified) + return (CdcCacheEvent)data.getValue(); + + return null; + }) + .filter(Objects::nonNull) + .iterator(); + + consumer.start(mreg, kctx.metric().registry(metricName("cdc", "consumer")), cacheEvts); started = true; diff --git a/modules/core/src/main/java/org/apache/ignite/internal/cdc/WalRecordsConsumer.java b/modules/core/src/main/java/org/apache/ignite/internal/cdc/WalRecordsConsumer.java index 3b111d50a197e..0d010aa0f0754 100644 --- a/modules/core/src/main/java/org/apache/ignite/internal/cdc/WalRecordsConsumer.java +++ b/modules/core/src/main/java/org/apache/ignite/internal/cdc/WalRecordsConsumer.java @@ -186,10 +186,15 @@ public void onCacheDestroyEvents(Iterator caches) { * * @param cdcReg CDC metric registry. * @param cdcConsumerReg CDC consumer metric registry. + * @param cacheEvents The iterator containing previously handled {@link CdcCacheEvent}s. * @throws IgniteCheckedException If failed. */ - public void start(MetricRegistryImpl cdcReg, MetricRegistryImpl cdcConsumerReg) throws IgniteCheckedException { - consumer.start(cdcConsumerReg); + public void start(MetricRegistryImpl cdcReg, MetricRegistryImpl cdcConsumerReg, Iterator cacheEvents) + throws IgniteCheckedException { + if (consumer instanceof CdcConsumerEx) + ((CdcConsumerEx)consumer).start(cdcConsumerReg, cacheEvents); + else + consumer.start(cdcConsumerReg); evtsCnt = cdcReg.longMetric(EVTS_CNT, "Count of events processed by the consumer"); lastEvtTs = cdcReg.longMetric(LAST_EVT_TIME, "Time of the last event process"); @@ -200,7 +205,7 @@ public void start(MetricRegistryImpl cdcReg, MetricRegistryImpl cdcConsumerReg) /** * Stops the consumer. - * This methods can be invoked only after {@link #start(MetricRegistryImpl, MetricRegistryImpl)}. + * This methods can be invoked only after {@link #start(MetricRegistryImpl, MetricRegistryImpl, Iterator)}. */ public void stop() { consumer.stop();