diff --git a/README.md b/README.md index 454ce27b0f5..f73a9755d06 100644 --- a/README.md +++ b/README.md @@ -125,7 +125,7 @@ $ docker run -it --net=host apache/rocketmq ./mqnamesrv **2) Start Broker** ```shell -$ docker run -it --net=host --mount source=/tmp/store,target=/home/rocketmq/store apache/rocketmq ./mqbroker -n localhost:9876 +$ docker run -it --net=host --mount type=bind,source=/tmp/store,target=/home/rocketmq/store apache/rocketmq ./mqbroker -n localhost:9876 ``` ### Run RocketMQ in Kubernetes diff --git a/auth/src/main/java/org/apache/rocketmq/auth/authorization/builder/DefaultAuthorizationContextBuilder.java b/auth/src/main/java/org/apache/rocketmq/auth/authorization/builder/DefaultAuthorizationContextBuilder.java index bf86892ea61..fababc0ee71 100644 --- a/auth/src/main/java/org/apache/rocketmq/auth/authorization/builder/DefaultAuthorizationContextBuilder.java +++ b/auth/src/main/java/org/apache/rocketmq/auth/authorization/builder/DefaultAuthorizationContextBuilder.java @@ -182,8 +182,13 @@ public List build(ChannelHandlerContext context, Re Resource group; switch (command.getCode()) { case RequestCode.GET_ROUTEINFO_BY_TOPIC: - topic = Resource.ofTopic(fields.get(TOPIC)); - result.add(DefaultAuthorizationContext.of(subject, topic, Arrays.asList(Action.PUB, Action.SUB, Action.GET), sourceIp)); + if (NamespaceUtil.isRetryTopic(fields.get(TOPIC))) { + group = Resource.ofGroup(fields.get(TOPIC)); + result.add(DefaultAuthorizationContext.of(subject, group, Arrays.asList(Action.SUB, Action.GET), sourceIp)); + } else { + topic = Resource.ofTopic(fields.get(TOPIC)); + result.add(DefaultAuthorizationContext.of(subject, topic, Arrays.asList(Action.PUB, Action.SUB, Action.GET), sourceIp)); + } break; case RequestCode.SEND_MESSAGE: if (NamespaceUtil.isRetryTopic(fields.get(TOPIC))) { diff --git a/broker/BUILD.bazel b/broker/BUILD.bazel index c21f9d114c3..77d456bc16a 100644 --- a/broker/BUILD.bazel +++ b/broker/BUILD.bazel @@ -30,6 +30,7 @@ java_library( "//srvutil", "//store", "//tieredstore", + "@maven//:org_slf4j_slf4j_api", "@maven//:ch_qos_logback_logback_classic", "@maven//:com_alibaba_fastjson", "@maven//:com_alibaba_fastjson2_fastjson2", diff --git a/broker/src/main/java/org/apache/rocketmq/broker/BrokerController.java b/broker/src/main/java/org/apache/rocketmq/broker/BrokerController.java index 744aba19118..006695c6bc8 100644 --- a/broker/src/main/java/org/apache/rocketmq/broker/BrokerController.java +++ b/broker/src/main/java/org/apache/rocketmq/broker/BrokerController.java @@ -79,6 +79,7 @@ import org.apache.rocketmq.broker.config.v1.RocksDBConsumerOffsetManager; import org.apache.rocketmq.broker.out.BrokerOuterAPI; import org.apache.rocketmq.broker.plugin.BrokerAttachedPlugin; +import org.apache.rocketmq.broker.pop.PopConsumerService; import org.apache.rocketmq.broker.processor.AckMessageProcessor; import org.apache.rocketmq.broker.processor.AdminBrokerProcessor; import org.apache.rocketmq.broker.processor.ChangeInvisibleTimeProcessor; @@ -198,6 +199,7 @@ public class BrokerController { protected final ConsumerFilterManager consumerFilterManager; protected final ConsumerOrderInfoManager consumerOrderInfoManager; protected final PopInflightMessageCounter popInflightMessageCounter; + protected final PopConsumerService popConsumerService; protected final ProducerManager producerManager; protected final ScheduleMessageService scheduleMessageService; protected final ClientHousekeepingService clientHousekeepingService; @@ -380,6 +382,7 @@ public BrokerController( this.consumerFilterManager = new ConsumerFilterManager(this); this.consumerOrderInfoManager = new ConsumerOrderInfoManager(this); this.popInflightMessageCounter = new PopInflightMessageCounter(this); + this.popConsumerService = brokerConfig.isPopConsumerKVServiceInit() ? new PopConsumerService(this) : null; this.clientHousekeepingService = new ClientHousekeepingService(this); this.broker2Client = new Broker2Client(this); this.scheduleMessageService = new ScheduleMessageService(this); @@ -1314,6 +1317,10 @@ public PopInflightMessageCounter getPopInflightMessageCounter() { return popInflightMessageCounter; } + public PopConsumerService getPopConsumerService() { + return popConsumerService; + } + public ConsumerOffsetManager getConsumerOffsetManager() { return consumerOffsetManager; } @@ -1417,12 +1424,13 @@ protected void shutdownBasicService() { this.pullRequestHoldService.shutdown(); } - { - this.popMessageProcessor.getPopLongPollingService().shutdown(); - this.popMessageProcessor.getQueueLockManager().shutdown(); + if (this.popConsumerService != null) { + this.popConsumerService.shutdown(); } { + this.popMessageProcessor.getPopLongPollingService().shutdown(); + this.popMessageProcessor.getQueueLockManager().shutdown(); this.popMessageProcessor.getPopBufferMergeService().shutdown(); this.ackMessageProcessor.shutdownPopReviveService(); } @@ -1673,18 +1681,26 @@ protected void startBasicService() throws Exception { if (this.popMessageProcessor != null) { this.popMessageProcessor.getPopLongPollingService().start(); - this.popMessageProcessor.getPopBufferMergeService().start(); + if (brokerConfig.isPopConsumerFSServiceInit()) { + this.popMessageProcessor.getPopBufferMergeService().start(); + } this.popMessageProcessor.getQueueLockManager().start(); } if (this.ackMessageProcessor != null) { - this.ackMessageProcessor.startPopReviveService(); + if (brokerConfig.isPopConsumerFSServiceInit()) { + this.ackMessageProcessor.startPopReviveService(); + } } if (this.notificationProcessor != null) { this.notificationProcessor.getPopLongPollingService().start(); } + if (this.popConsumerService != null) { + this.popConsumerService.start(); + } + if (this.topicQueueMappingCleanService != null) { this.topicQueueMappingCleanService.start(); } diff --git a/broker/src/main/java/org/apache/rocketmq/broker/coldctr/ColdDataCgCtrService.java b/broker/src/main/java/org/apache/rocketmq/broker/coldctr/ColdDataCgCtrService.java index dd9278fb755..2e249304056 100644 --- a/broker/src/main/java/org/apache/rocketmq/broker/coldctr/ColdDataCgCtrService.java +++ b/broker/src/main/java/org/apache/rocketmq/broker/coldctr/ColdDataCgCtrService.java @@ -24,8 +24,7 @@ import java.util.concurrent.ConcurrentHashMap; import java.util.concurrent.atomic.AtomicLong; -import com.alibaba.fastjson.JSONObject; - +import com.alibaba.fastjson2.JSONObject; import org.apache.rocketmq.broker.BrokerController; import org.apache.rocketmq.common.BrokerConfig; import org.apache.rocketmq.common.MixAll; diff --git a/broker/src/main/java/org/apache/rocketmq/broker/config/v1/RocksDBConsumerOffsetManager.java b/broker/src/main/java/org/apache/rocketmq/broker/config/v1/RocksDBConsumerOffsetManager.java index 824fc0fee3e..963c5046f24 100644 --- a/broker/src/main/java/org/apache/rocketmq/broker/config/v1/RocksDBConsumerOffsetManager.java +++ b/broker/src/main/java/org/apache/rocketmq/broker/config/v1/RocksDBConsumerOffsetManager.java @@ -38,7 +38,7 @@ public class RocksDBConsumerOffsetManager extends ConsumerOffsetManager { protected static final Logger log = LoggerFactory.getLogger(LoggerName.BROKER_LOGGER_NAME); - protected RocksDBConfigManager rocksDBConfigManager; + protected transient RocksDBConfigManager rocksDBConfigManager; public RocksDBConsumerOffsetManager(BrokerController brokerController) { super(brokerController); @@ -100,7 +100,7 @@ protected void removeConsumerOffset(String topicAtGroup) { byte[] keyBytes = topicAtGroup.getBytes(DataConverter.CHARSET_UTF8); this.rocksDBConfigManager.delete(keyBytes); } catch (Exception e) { - LOG.error("kv remove consumerOffset Failed, {}", topicAtGroup); + log.error("kv remove consumerOffset Failed, {}", topicAtGroup); } } @@ -109,7 +109,7 @@ protected void decodeOffset(final byte[] key, final byte[] body) { RocksDBOffsetSerializeWrapper wrapper = JSON.parseObject(body, RocksDBOffsetSerializeWrapper.class); this.offsetTable.put(topicAtGroup, wrapper.getOffsetTable()); - LOG.info("load exist local offset, {}, {}", topicAtGroup, wrapper.getOffsetTable()); + log.info("load exist local offset, {}, {}", topicAtGroup, wrapper.getOffsetTable()); } public String rocksdbConfigFilePath() { @@ -132,12 +132,17 @@ public synchronized void persist() { this.rocksDBConfigManager.batchPutWithWal(writeBatch); this.rocksDBConfigManager.flushWAL(); } catch (Exception e) { - LOG.error("consumer offset persist Failed", e); + log.error("consumer offset persist Failed", e); } finally { writeBatch.close(); } } + public synchronized void exportToJson() { + log.info("RocksDBConsumerOffsetManager export consumer offset to json file"); + super.persist(); + } + private void putWriteBatch(final WriteBatch writeBatch, final String topicGroupName, final ConcurrentMap offsetMap) throws Exception { byte[] keyBytes = topicGroupName.getBytes(DataConverter.CHARSET_UTF8); RocksDBOffsetSerializeWrapper wrapper = new RocksDBOffsetSerializeWrapper(); diff --git a/broker/src/main/java/org/apache/rocketmq/broker/config/v1/RocksDBSubscriptionGroupManager.java b/broker/src/main/java/org/apache/rocketmq/broker/config/v1/RocksDBSubscriptionGroupManager.java index 8fc7a4d6edb..ff471525691 100644 --- a/broker/src/main/java/org/apache/rocketmq/broker/config/v1/RocksDBSubscriptionGroupManager.java +++ b/broker/src/main/java/org/apache/rocketmq/broker/config/v1/RocksDBSubscriptionGroupManager.java @@ -37,7 +37,7 @@ public class RocksDBSubscriptionGroupManager extends SubscriptionGroupManager { - protected RocksDBConfigManager rocksDBConfigManager; + protected transient RocksDBConfigManager rocksDBConfigManager; public RocksDBSubscriptionGroupManager(BrokerController brokerController) { super(brokerController, false); @@ -184,6 +184,11 @@ public synchronized void persist() { } } + public synchronized void exportToJson() { + log.info("RocksDBSubscriptionGroupManager export subscription group to json file"); + super.persist(); + } + public String rocksdbConfigFilePath() { return this.brokerController.getMessageStoreConfig().getStorePathRootDir() + File.separator + "config" + File.separator + "subscriptionGroups" + File.separator; } diff --git a/broker/src/main/java/org/apache/rocketmq/broker/config/v1/RocksDBTopicConfigManager.java b/broker/src/main/java/org/apache/rocketmq/broker/config/v1/RocksDBTopicConfigManager.java index 18e633d348b..d64f808067c 100644 --- a/broker/src/main/java/org/apache/rocketmq/broker/config/v1/RocksDBTopicConfigManager.java +++ b/broker/src/main/java/org/apache/rocketmq/broker/config/v1/RocksDBTopicConfigManager.java @@ -32,7 +32,7 @@ public class RocksDBTopicConfigManager extends TopicConfigManager { - protected RocksDBConfigManager rocksDBConfigManager; + protected transient RocksDBConfigManager rocksDBConfigManager; public RocksDBTopicConfigManager(BrokerController brokerController) { super(brokerController, false); @@ -139,6 +139,11 @@ public synchronized void persist() { } } + public synchronized void exportToJson() { + log.info("RocksDBTopicConfigManager export topic config to json file"); + super.persist(); + } + public String rocksdbConfigFilePath() { return this.brokerController.getMessageStoreConfig().getStorePathRootDir() + File.separator + "config" + File.separator + "topics" + File.separator; } diff --git a/broker/src/main/java/org/apache/rocketmq/broker/longpolling/PopLongPollingService.java b/broker/src/main/java/org/apache/rocketmq/broker/longpolling/PopLongPollingService.java index 91185fbe94c..e87a8e803fd 100644 --- a/broker/src/main/java/org/apache/rocketmq/broker/longpolling/PopLongPollingService.java +++ b/broker/src/main/java/org/apache/rocketmq/broker/longpolling/PopLongPollingService.java @@ -52,7 +52,7 @@ public class PopLongPollingService extends ServiceThread { LoggerFactory.getLogger(LoggerName.ROCKETMQ_POP_LOGGER_NAME); private final BrokerController brokerController; private final NettyRequestProcessor processor; - private final ConcurrentHashMap> topicCidMap; + private final ConcurrentLinkedHashMap> topicCidMap; private final ConcurrentLinkedHashMap> pollingMap; private long lastCleanTime = 0; @@ -63,7 +63,8 @@ public PopLongPollingService(BrokerController brokerController, NettyRequestProc this.brokerController = brokerController; this.processor = processor; // 100000 topic default, 100000 lru topic + cid + qid - this.topicCidMap = new ConcurrentHashMap<>(brokerController.getBrokerConfig().getPopPollingMapSize()); + this.topicCidMap = new ConcurrentLinkedHashMap.Builder>() + .maximumWeightedCapacity(this.brokerController.getBrokerConfig().getPopPollingMapSize() * 2L).build(); this.pollingMap = new ConcurrentLinkedHashMap.Builder>() .maximumWeightedCapacity(this.brokerController.getBrokerConfig().getPopPollingMapSize()).build(); this.notifyLast = notifyLast; @@ -350,7 +351,7 @@ private void cleanUnusedResource() { Map.Entry> entry = topicCidMapIter.next(); String topic = entry.getKey(); if (brokerController.getTopicConfigManager().selectTopicConfig(topic) == null) { - POP_LOGGER.info("remove not exit topic {} in topicCidMap!", topic); + POP_LOGGER.info("remove nonexistent topic {} in topicCidMap!", topic); topicCidMapIter.remove(); continue; } @@ -358,8 +359,8 @@ private void cleanUnusedResource() { while (cidMapIter.hasNext()) { Map.Entry cidEntry = cidMapIter.next(); String cid = cidEntry.getKey(); - if (!brokerController.getSubscriptionGroupManager().getSubscriptionGroupTable().containsKey(cid)) { - POP_LOGGER.info("remove not exit sub {} of topic {} in topicCidMap!", cid, topic); + if (!brokerController.getSubscriptionGroupManager().containsSubscriptionGroup(cid)) { + POP_LOGGER.info("remove nonexistent subscription group {} of topic {} in topicCidMap!", cid, topic); cidMapIter.remove(); } } @@ -380,12 +381,12 @@ private void cleanUnusedResource() { String topic = keyArray[0]; String cid = keyArray[1]; if (brokerController.getTopicConfigManager().selectTopicConfig(topic) == null) { - POP_LOGGER.info("remove not exit topic {} in pollingMap!", topic); + POP_LOGGER.info("remove nonexistent topic {} in pollingMap!", topic); pollingMapIter.remove(); continue; } - if (!brokerController.getSubscriptionGroupManager().getSubscriptionGroupTable().containsKey(cid)) { - POP_LOGGER.info("remove not exit sub {} of topic {} in pollingMap!", cid, topic); + if (!brokerController.getSubscriptionGroupManager().containsSubscriptionGroup(cid)) { + POP_LOGGER.info("remove nonexistent subscription group {} of topic {} in pollingMap!", cid, topic); pollingMapIter.remove(); } } diff --git a/broker/src/main/java/org/apache/rocketmq/broker/offset/ConsumerOrderInfoManager.java b/broker/src/main/java/org/apache/rocketmq/broker/offset/ConsumerOrderInfoManager.java index 4eccc6c0374..120f5b104c7 100644 --- a/broker/src/main/java/org/apache/rocketmq/broker/offset/ConsumerOrderInfoManager.java +++ b/broker/src/main/java/org/apache/rocketmq/broker/offset/ConsumerOrderInfoManager.java @@ -281,7 +281,7 @@ protected void autoClean() { continue; } - if (this.brokerController.getSubscriptionGroupManager().getSubscriptionGroupTable().get(group) == null) { + if (!this.brokerController.getSubscriptionGroupManager().containsSubscriptionGroup(group)) { iterator.remove(); log.info("Group not exist, Clean order info, {}:{}", topicAtGroup, qs); continue; diff --git a/broker/src/main/java/org/apache/rocketmq/broker/pop/PopConsumerCache.java b/broker/src/main/java/org/apache/rocketmq/broker/pop/PopConsumerCache.java new file mode 100644 index 00000000000..e7ce68e0193 --- /dev/null +++ b/broker/src/main/java/org/apache/rocketmq/broker/pop/PopConsumerCache.java @@ -0,0 +1,303 @@ +/* + * 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.rocketmq.broker.pop; + +import java.util.ArrayList; +import java.util.Iterator; +import java.util.List; +import java.util.Map; +import java.util.TreeMap; +import java.util.concurrent.ConcurrentHashMap; +import java.util.concurrent.ConcurrentMap; +import java.util.concurrent.TimeUnit; +import java.util.concurrent.atomic.AtomicInteger; +import java.util.concurrent.locks.Lock; +import java.util.concurrent.locks.ReentrantLock; +import java.util.function.Consumer; +import org.apache.rocketmq.broker.BrokerController; +import org.apache.rocketmq.broker.offset.ConsumerOffsetManager; +import org.apache.rocketmq.common.BrokerConfig; +import org.apache.rocketmq.common.ServiceThread; +import org.apache.rocketmq.common.constant.LoggerName; +import org.apache.rocketmq.common.utils.ConcurrentHashMapUtils; +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; + +public class PopConsumerCache extends ServiceThread { + + private static final Logger log = LoggerFactory.getLogger(LoggerName.ROCKETMQ_POP_LOGGER_NAME); + private static final long OFFSET_NOT_EXIST = -1L; + + private final BrokerController brokerController; + private final PopConsumerKVStore consumerRecordStore; + private final PopConsumerLockService consumerLockService; + private final Consumer reviveConsumer; + + private final AtomicInteger estimateCacheSize; + private final ConcurrentMap consumerRecordTable; + + public PopConsumerCache(BrokerController brokerController, PopConsumerKVStore consumerRecordStore, + PopConsumerLockService popConsumerLockService, Consumer reviveConsumer) { + + this.reviveConsumer = reviveConsumer; + this.brokerController = brokerController; + this.consumerRecordStore = consumerRecordStore; + this.consumerLockService = popConsumerLockService; + this.estimateCacheSize = new AtomicInteger(); + this.consumerRecordTable = new ConcurrentHashMap<>(); + } + + public String getKey(String groupId, String topicId, int queueId) { + return groupId + "@" + topicId + "@" + queueId; + } + + public String getKey(PopConsumerRecord consumerRecord) { + return consumerRecord.getGroupId() + "@" + consumerRecord.getTopicId() + "@" + consumerRecord.getQueueId(); + } + + public int getCacheKeySize() { + return this.consumerRecordTable.size(); + } + + public int getCacheSize() { + return this.estimateCacheSize.intValue(); + } + + public boolean isCacheFull() { + return this.estimateCacheSize.intValue() > brokerController.getBrokerConfig().getPopCkMaxBufferSize(); + } + + public long getMinOffsetInCache(String groupId, String topicId, int queueId) { + ConsumerRecords consumerRecords = consumerRecordTable.get(this.getKey(groupId, topicId, queueId)); + return consumerRecords != null ? consumerRecords.getMinOffsetInBuffer() : OFFSET_NOT_EXIST; + } + + public long getPopInFlightMessageCount(String groupId, String topicId, int queueId) { + ConsumerRecords consumerRecords = consumerRecordTable.get(this.getKey(groupId, topicId, queueId)); + return consumerRecords != null ? consumerRecords.getInFlightRecordCount() : 0L; + } + + public void writeRecords(List consumerRecordList) { + this.estimateCacheSize.addAndGet(consumerRecordList.size()); + consumerRecordList.forEach(consumerRecord -> { + ConsumerRecords consumerRecords = ConcurrentHashMapUtils.computeIfAbsent(consumerRecordTable, + this.getKey(consumerRecord), k -> new ConsumerRecords(brokerController.getBrokerConfig(), + consumerRecord.getGroupId(), consumerRecord.getTopicId(), consumerRecord.getQueueId())); + assert consumerRecords != null; + consumerRecords.write(consumerRecord); + }); + } + + /** + * Remove the record from the input list then return the content that has not been deleted + */ + public List deleteRecords(List consumerRecordList) { + int total = consumerRecordList.size(); + List remain = new ArrayList<>(); + consumerRecordList.forEach(consumerRecord -> { + ConsumerRecords consumerRecords = consumerRecordTable.get(this.getKey(consumerRecord)); + if (consumerRecords == null || !consumerRecords.delete(consumerRecord)) { + remain.add(consumerRecord); + } + }); + this.estimateCacheSize.addAndGet(remain.size() - total); + return remain; + } + + public int cleanupRecords(Consumer consumer) { + int remain = 0; + Iterator> iterator = consumerRecordTable.entrySet().iterator(); + while (iterator.hasNext()) { + // revive or write record to store + ConsumerRecords records = iterator.next().getValue(); + boolean timeout = consumerLockService.isLockTimeout( + records.getGroupId(), records.getTopicId()); + + if (timeout) { + List removeExpiredRecords = + records.removeExpiredRecords(Long.MAX_VALUE); + if (removeExpiredRecords != null) { + consumerRecordStore.writeRecords(removeExpiredRecords); + } + log.info("PopConsumerOffline, so clean expire records, groupId={}, topic={}, queueId={}, records={}", + records.getGroupId(), records.getTopicId(), records.getQueueId(), + removeExpiredRecords != null ? removeExpiredRecords.size() : 0); + iterator.remove(); + continue; + } + + long currentTime = System.currentTimeMillis(); + List writeConsumerRecords = new ArrayList<>(); + List consumerRecords = records.removeExpiredRecords(currentTime); + if (consumerRecords != null) { + consumerRecords.forEach(consumerRecord -> { + if (consumerRecord.getVisibilityTimeout() <= currentTime) { + consumer.accept(consumerRecord); + } else { + writeConsumerRecords.add(consumerRecord); + } + }); + } + + // write to store and handle it later + consumerRecordStore.writeRecords(writeConsumerRecords); + + // commit min offset in buffer to offset store + long offset = records.getMinOffsetInBuffer(); + if (offset > OFFSET_NOT_EXIST) { + this.commitOffset("PopConsumerCache", + records.getGroupId(), records.getTopicId(), records.getQueueId(), offset); + } + + remain += records.getInFlightRecordCount(); + } + return remain; + } + + public void commitOffset(String clientHost, String groupId, String topicId, int queueId, long offset) { + if (!consumerLockService.tryLock(groupId, topicId)) { + return; + } + try { + ConsumerOffsetManager consumerOffsetManager = brokerController.getConsumerOffsetManager(); + long commit = consumerOffsetManager.queryOffset(groupId, topicId, queueId); + if (commit != OFFSET_NOT_EXIST && offset < commit) { + log.info("PopConsumerCache, consumer offset less than store, " + + "groupId={}, topicId={}, queueId={}, offset={}", groupId, topicId, queueId, offset); + } + consumerOffsetManager.commitOffset(clientHost, groupId, topicId, queueId, offset); + } finally { + consumerLockService.unlock(groupId, topicId); + } + } + + public void removeRecords(String groupId, String topicId, int queueId) { + this.consumerRecordTable.remove(this.getKey(groupId, topicId, queueId)); + } + + @Override + public String getServiceName() { + return PopConsumerCache.class.getSimpleName(); + } + + @Override + public void run() { + while (!this.isStopped()) { + try { + this.waitForRunning(TimeUnit.SECONDS.toMillis(1)); + int cacheSize = this.cleanupRecords(reviveConsumer); + this.estimateCacheSize.set(cacheSize); + } catch (Exception e) { + log.error("PopConsumerCacheService revive error", e); + } + } + } + + protected static class ConsumerRecords { + + private final Lock lock; + private final String groupId; + private final String topicId; + private final int queueId; + private final BrokerConfig brokerConfig; + private final TreeMap recordTreeMap; + + public ConsumerRecords(BrokerConfig brokerConfig, String groupId, String topicId, int queueId) { + this.groupId = groupId; + this.topicId = topicId; + this.queueId = queueId; + this.lock = new ReentrantLock(); + this.brokerConfig = brokerConfig; + this.recordTreeMap = new TreeMap<>(); + } + + public void write(PopConsumerRecord record) { + lock.lock(); + try { + recordTreeMap.put(record.getOffset(), record); + } finally { + lock.unlock(); + } + } + + public boolean delete(PopConsumerRecord record) { + PopConsumerRecord popConsumerRecord; + lock.lock(); + try { + popConsumerRecord = recordTreeMap.remove(record.getOffset()); + } finally { + lock.unlock(); + } + return popConsumerRecord != null; + } + + public long getMinOffsetInBuffer() { + Map.Entry entry = recordTreeMap.firstEntry(); + return entry != null ? entry.getKey() : OFFSET_NOT_EXIST; + } + + public int getInFlightRecordCount() { + return recordTreeMap.size(); + } + + public List removeExpiredRecords(long currentTime) { + List result = null; + lock.lock(); + try { + Iterator> iterator = recordTreeMap.entrySet().iterator(); + while (iterator.hasNext()) { + Map.Entry entry = iterator.next(); + // org.apache.rocketmq.broker.processor.PopBufferMergeService.scan + if (entry.getValue().getVisibilityTimeout() <= currentTime || + entry.getValue().getPopTime() + brokerConfig.getPopCkStayBufferTime() <= currentTime) { + if (result == null) { + result = new ArrayList<>(); + } + result.add(entry.getValue()); + iterator.remove(); + } + } + } finally { + lock.unlock(); + } + return result; + } + + public String getGroupId() { + return groupId; + } + + public String getTopicId() { + return topicId; + } + + public int getQueueId() { + return queueId; + } + + @Override + public String toString() { + return "ConsumerRecords{" + + "lock=" + lock + + ", topicId=" + topicId + + ", groupId=" + groupId + + ", queueId=" + queueId + + ", recordTreeMap=" + recordTreeMap.size() + + '}'; + } + } +} diff --git a/broker/src/main/java/org/apache/rocketmq/broker/pop/PopConsumerContext.java b/broker/src/main/java/org/apache/rocketmq/broker/pop/PopConsumerContext.java new file mode 100644 index 00000000000..09bc4e6b47c --- /dev/null +++ b/broker/src/main/java/org/apache/rocketmq/broker/pop/PopConsumerContext.java @@ -0,0 +1,177 @@ +/* + * 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.rocketmq.broker.pop; + +import java.util.ArrayList; +import java.util.List; +import java.util.concurrent.atomic.AtomicLong; +import org.apache.rocketmq.remoting.protocol.header.ExtraInfoUtil; +import org.apache.rocketmq.store.GetMessageResult; +import org.apache.rocketmq.store.GetMessageStatus; + +public class PopConsumerContext { + + private final String clientHost; + + private final long popTime; + + private final long invisibleTime; + + private final String groupId; + + private final boolean fifo; + + private final String attemptId; + + private final AtomicLong restCount; + + private final StringBuilder startOffsetInfo; + + private final StringBuilder msgOffsetInfo; + + private final StringBuilder orderCountInfo; + + private List getMessageResultList; + + private List popConsumerRecordList; + + public PopConsumerContext(String clientHost, + long popTime, long invisibleTime, String groupId, boolean fifo, String attemptId) { + + this.clientHost = clientHost; + this.popTime = popTime; + this.invisibleTime = invisibleTime; + this.groupId = groupId; + this.fifo = fifo; + this.attemptId = attemptId; + this.restCount = new AtomicLong(0); + this.startOffsetInfo = new StringBuilder(); + this.msgOffsetInfo = new StringBuilder(); + this.orderCountInfo = new StringBuilder(); + } + + public boolean isFound() { + return getMessageResultList != null && !getMessageResultList.isEmpty(); + } + + // offset is consumer last request offset + public void addGetMessageResult(GetMessageResult result, + String topicId, int queueId, PopConsumerRecord.RetryType retryType, long offset) { + + if (result.getStatus() != GetMessageStatus.FOUND || result.getMessageQueueOffset().isEmpty()) { + return; + } + + if (this.getMessageResultList == null) { + this.getMessageResultList = new ArrayList<>(); + } + + if (this.popConsumerRecordList == null) { + this.popConsumerRecordList = new ArrayList<>(); + } + + this.getMessageResultList.add(result); + this.addRestCount(result.getMaxOffset() - result.getNextBeginOffset()); + + for (int i = 0; i < result.getMessageQueueOffset().size(); i++) { + this.popConsumerRecordList.add(new PopConsumerRecord(popTime, groupId, topicId, queueId, + retryType.getCode(), invisibleTime, result.getMessageQueueOffset().get(i), attemptId)); + } + + ExtraInfoUtil.buildStartOffsetInfo(startOffsetInfo, topicId, queueId, offset); + ExtraInfoUtil.buildMsgOffsetInfo(msgOffsetInfo, topicId, queueId, result.getMessageQueueOffset()); + } + + public String getClientHost() { + return clientHost; + } + + public String getGroupId() { + return groupId; + } + + public void addRestCount(long delta) { + this.restCount.addAndGet(delta); + } + + public long getRestCount() { + return restCount.get(); + } + + public long getPopTime() { + return popTime; + } + + public boolean isFifo() { + return fifo; + } + + public long getInvisibleTime() { + return invisibleTime; + } + + public String getAttemptId() { + return attemptId; + } + + public int getMessageCount() { + return getMessageResultList != null ? + getMessageResultList.stream().mapToInt(GetMessageResult::getMessageCount).sum() : 0; + } + + public String getStartOffsetInfo() { + return startOffsetInfo.toString(); + } + + public String getMsgOffsetInfo() { + return msgOffsetInfo.toString(); + } + + public StringBuilder getOrderCountInfoBuilder() { + return orderCountInfo; + } + + public String getOrderCountInfo() { + return orderCountInfo.toString(); + } + + public List getGetMessageResultList() { + return getMessageResultList; + } + + public List getPopConsumerRecordList() { + return popConsumerRecordList; + } + + @Override + public String toString() { + return "PopConsumerContext{" + + "clientHost=" + clientHost + + ", popTime=" + popTime + + ", invisibleTime=" + invisibleTime + + ", groupId=" + groupId + + ", isFifo=" + fifo + + ", attemptId=" + attemptId + + ", restCount=" + restCount + + ", startOffsetInfo=" + startOffsetInfo + + ", msgOffsetInfo=" + msgOffsetInfo + + ", orderCountInfo=" + orderCountInfo + + ", getMessageResultList=" + (getMessageResultList != null ? getMessageResultList.size() : 0) + + ", popConsumerRecordList=" + (popConsumerRecordList != null ? popConsumerRecordList.size() : 0) + + '}'; + } +} diff --git a/broker/src/main/java/org/apache/rocketmq/broker/pop/PopConsumerKVStore.java b/broker/src/main/java/org/apache/rocketmq/broker/pop/PopConsumerKVStore.java new file mode 100644 index 00000000000..33072d699b5 --- /dev/null +++ b/broker/src/main/java/org/apache/rocketmq/broker/pop/PopConsumerKVStore.java @@ -0,0 +1,61 @@ +/* + * 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.rocketmq.broker.pop; + +import java.util.List; + +public interface PopConsumerKVStore { + + /** + * Starts the storage service. + */ + boolean start(); + + /** + * Shutdown the storage service. + */ + boolean shutdown(); + + /** + * Gets the file path of the storage. + * @return The file path of the storage. + */ + String getFilePath(); + + /** + * Writes a list of consumer records to the storage. + * @param consumerRecordList The list of consumer records to be written. + */ + void writeRecords(List consumerRecordList); + + /** + * Deletes a list of consumer records from the storage. + * @param consumerRecordList The list of consumer records to be deleted. + */ + void deleteRecords(List consumerRecordList); + + /** + * Scans and returns a list of expired consumer records within the specified time range. + * @param lowerTime The start time (inclusive) of the time range to search, in milliseconds. + * @param upperTime The end time (exclusive) of the time range to search, in milliseconds. + * @param maxCount The maximum number of records to return. + * Even if more records match the criteria, only this many will be returned. + * @return A list of expired consumer records within the specified time range. + * If no matching records are found, an empty list is returned. + */ + List scanExpiredRecords(long lowerTime, long upperTime, int maxCount); +} diff --git a/broker/src/main/java/org/apache/rocketmq/broker/pop/PopConsumerLockService.java b/broker/src/main/java/org/apache/rocketmq/broker/pop/PopConsumerLockService.java new file mode 100644 index 00000000000..33221430492 --- /dev/null +++ b/broker/src/main/java/org/apache/rocketmq/broker/pop/PopConsumerLockService.java @@ -0,0 +1,100 @@ +/* + * 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.rocketmq.broker.pop; + +import java.util.Iterator; +import java.util.Map; +import java.util.Objects; +import java.util.concurrent.ConcurrentHashMap; +import java.util.concurrent.ConcurrentMap; +import java.util.concurrent.atomic.AtomicBoolean; +import org.apache.rocketmq.common.KeyBuilder; +import org.apache.rocketmq.common.PopAckConstants; +import org.apache.rocketmq.common.constant.LoggerName; +import org.apache.rocketmq.common.utils.ConcurrentHashMapUtils; +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; + +public class PopConsumerLockService { + + private static final Logger log = LoggerFactory.getLogger(LoggerName.ROCKETMQ_POP_LOGGER_NAME); + + private final long timeout; + private final ConcurrentMap lockTable; + + public PopConsumerLockService(long timeout) { + this.timeout = timeout; + this.lockTable = new ConcurrentHashMap<>(); + } + + public boolean tryLock(String groupId, String topicId) { + return Objects.requireNonNull(ConcurrentHashMapUtils.computeIfAbsent(lockTable, + groupId + PopAckConstants.SPLIT + topicId, s -> new TimedLock())).tryLock(); + } + + public void unlock(String groupId, String topicId) { + TimedLock lock = lockTable.get(groupId + PopAckConstants.SPLIT + topicId); + if (lock != null) { + lock.unlock(); + } + } + + // For retry topics, should lock origin group and topic + public boolean isLockTimeout(String groupId, String topicId) { + topicId = KeyBuilder.parseNormalTopic(topicId, groupId); + TimedLock lock = lockTable.get(groupId + PopAckConstants.SPLIT + topicId); + return lock == null || System.currentTimeMillis() - lock.getLockTime() > timeout; + } + + public void removeTimeout() { + Iterator> iterator = lockTable.entrySet().iterator(); + while (iterator.hasNext()) { + Map.Entry entry = iterator.next(); + if (System.currentTimeMillis() - entry.getValue().getLockTime() > timeout) { + log.info("PopConsumerLockService remove timeout lock, " + + "key={}, locked={}", entry.getKey(), entry.getValue().lock.get()); + iterator.remove(); + } + } + } + + static class TimedLock { + private volatile long lockTime; + private final AtomicBoolean lock; + + public TimedLock() { + this.lockTime = System.currentTimeMillis(); + this.lock = new AtomicBoolean(false); + } + + public boolean tryLock() { + if (lock.compareAndSet(false, true)) { + this.lockTime = System.currentTimeMillis(); + return true; + } + return false; + } + + public void unlock() { + lock.set(false); + } + + public long getLockTime() { + return lockTime; + } + } +} \ No newline at end of file diff --git a/broker/src/main/java/org/apache/rocketmq/broker/pop/PopConsumerRecord.java b/broker/src/main/java/org/apache/rocketmq/broker/pop/PopConsumerRecord.java new file mode 100644 index 00000000000..1ee01fea1c8 --- /dev/null +++ b/broker/src/main/java/org/apache/rocketmq/broker/pop/PopConsumerRecord.java @@ -0,0 +1,211 @@ +/* + * 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.rocketmq.broker.pop; + +import com.alibaba.fastjson.JSON; +import com.alibaba.fastjson.JSONObject; +import com.alibaba.fastjson.annotation.JSONField; +import java.nio.ByteBuffer; +import java.nio.charset.StandardCharsets; + +public class PopConsumerRecord { + + public enum RetryType { + + NORMAL_TOPIC(0), + + RETRY_TOPIC_V1(1), + + RETRY_TOPIC_V2(2); + + private final int code; + + RetryType(int code) { + this.code = code; + } + + public int getCode() { + return code; + } + } + + @JSONField() + private long popTime; + + @JSONField(ordinal = 1) + private String groupId; + + @JSONField(ordinal = 2) + private String topicId; + + @JSONField(ordinal = 3) + private int queueId; + + @JSONField(ordinal = 4) + private int retryFlag; + + @JSONField(ordinal = 5) + private long invisibleTime; + + @JSONField(ordinal = 6) + private long offset; + + @JSONField(ordinal = 7) + private int attemptTimes; + + @JSONField(ordinal = 8) + private String attemptId; + + // used for test and fastjson + public PopConsumerRecord() { + } + + public PopConsumerRecord(long popTime, String groupId, String topicId, int queueId, + int retryFlag, long invisibleTime, long offset, String attemptId) { + + this.popTime = popTime; + this.groupId = groupId; + this.topicId = topicId; + this.queueId = queueId; + this.retryFlag = retryFlag; + this.invisibleTime = invisibleTime; + this.offset = offset; + this.attemptId = attemptId; + } + + @JSONField(serialize = false) + public long getVisibilityTimeout() { + return popTime + invisibleTime; + } + + /** + * Key: timestamp(8) + groupId + topicId + queueId + offset + */ + @JSONField(serialize = false) + public byte[] getKeyBytes() { + int length = Long.BYTES + groupId.length() + 1 + topicId.length() + 1 + Integer.BYTES + 1 + Long.BYTES; + byte[] bytes = new byte[length]; + ByteBuffer buffer = ByteBuffer.wrap(bytes); + buffer.putLong(this.getVisibilityTimeout()); + buffer.put(groupId.getBytes(StandardCharsets.UTF_8)).put((byte) '@'); + buffer.put(topicId.getBytes(StandardCharsets.UTF_8)).put((byte) '@'); + buffer.putInt(queueId).put((byte) '@'); + buffer.putLong(offset); + return bytes; + } + + @JSONField(serialize = false) + public boolean isRetry() { + return retryFlag != 0; + } + + @JSONField(serialize = false) + public byte[] getValueBytes() { + return JSON.toJSONBytes(this); + } + + public static PopConsumerRecord decode(byte[] body) { + return JSONObject.parseObject(body, PopConsumerRecord.class); + } + + public long getPopTime() { + return popTime; + } + + public void setPopTime(long popTime) { + this.popTime = popTime; + } + + public String getGroupId() { + return groupId; + } + + public void setGroupId(String groupId) { + this.groupId = groupId; + } + + public String getTopicId() { + return topicId; + } + + public void setTopicId(String topicId) { + this.topicId = topicId; + } + + public int getQueueId() { + return queueId; + } + + public void setQueueId(int queueId) { + this.queueId = queueId; + } + + public int getRetryFlag() { + return retryFlag; + } + + public void setRetryFlag(int retryFlag) { + this.retryFlag = retryFlag; + } + + public long getInvisibleTime() { + return invisibleTime; + } + + public void setInvisibleTime(long invisibleTime) { + this.invisibleTime = invisibleTime; + } + + public long getOffset() { + return offset; + } + + public void setOffset(long offset) { + this.offset = offset; + } + + public int getAttemptTimes() { + return attemptTimes; + } + + public void setAttemptTimes(int attemptTimes) { + this.attemptTimes = attemptTimes; + } + + public String getAttemptId() { + return attemptId; + } + + public void setAttemptId(String attemptId) { + this.attemptId = attemptId; + } + + @Override + public String toString() { + return "PopDeliveryRecord{" + + "popTime=" + popTime + + ", groupId='" + groupId + '\'' + + ", topicId='" + topicId + '\'' + + ", queueId=" + queueId + + ", retryFlag=" + retryFlag + + ", invisibleTime=" + invisibleTime + + ", offset=" + offset + + ", attemptTimes=" + attemptTimes + + ", attemptId='" + attemptId + '\'' + + '}'; + } +} diff --git a/broker/src/main/java/org/apache/rocketmq/broker/pop/PopConsumerRocksdbStore.java b/broker/src/main/java/org/apache/rocketmq/broker/pop/PopConsumerRocksdbStore.java new file mode 100644 index 00000000000..7ab276a4185 --- /dev/null +++ b/broker/src/main/java/org/apache/rocketmq/broker/pop/PopConsumerRocksdbStore.java @@ -0,0 +1,171 @@ +/* + * 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.rocketmq.broker.pop; + +import java.nio.ByteBuffer; +import java.nio.charset.StandardCharsets; +import java.util.ArrayList; +import java.util.List; +import org.apache.rocketmq.common.UtilAll; +import org.apache.rocketmq.common.config.AbstractRocksDBStorage; +import org.apache.rocketmq.common.constant.LoggerName; +import org.apache.rocketmq.store.rocksdb.RocksDBOptionsFactory; +import org.rocksdb.ColumnFamilyDescriptor; +import org.rocksdb.ColumnFamilyHandle; +import org.rocksdb.ColumnFamilyOptions; +import org.rocksdb.CompactRangeOptions; +import org.rocksdb.ReadOptions; +import org.rocksdb.RocksDB; +import org.rocksdb.RocksDBException; +import org.rocksdb.RocksIterator; +import org.rocksdb.Slice; +import org.rocksdb.WriteBatch; +import org.rocksdb.WriteOptions; +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; + +public class PopConsumerRocksdbStore extends AbstractRocksDBStorage implements PopConsumerKVStore { + + private static final Logger log = LoggerFactory.getLogger(LoggerName.ROCKETMQ_POP_LOGGER_NAME); + private static final byte[] COLUMN_FAMILY_NAME = "popState".getBytes(StandardCharsets.UTF_8); + + private WriteOptions writeOptions; + private WriteOptions deleteOptions; + protected ColumnFamilyHandle columnFamilyHandle; + + public PopConsumerRocksdbStore(String filePath) { + super(filePath); + } + + // https://www.cnblogs.com/renjc/p/rocksdb-class-db.html + // https://github.com/johnzeng/rocksdb-doc-cn/blob/master/doc/RocksDB-Tuning-Guide.md + protected void initOptions() { + this.options = RocksDBOptionsFactory.createDBOptions(); + + this.writeOptions = new WriteOptions(); + this.writeOptions.setSync(true); + this.writeOptions.setDisableWAL(false); + this.writeOptions.setNoSlowdown(false); + + this.deleteOptions = new WriteOptions(); + this.deleteOptions.setSync(true); + this.deleteOptions.setDisableWAL(false); + this.deleteOptions.setNoSlowdown(false); + + this.compactRangeOptions = new CompactRangeOptions(); + this.compactRangeOptions.setBottommostLevelCompaction( + CompactRangeOptions.BottommostLevelCompaction.kForce); + this.compactRangeOptions.setAllowWriteStall(true); + this.compactRangeOptions.setExclusiveManualCompaction(false); + this.compactRangeOptions.setChangeLevel(true); + this.compactRangeOptions.setTargetLevel(-1); + this.compactRangeOptions.setMaxSubcompactions(4); + } + + @Override + protected boolean postLoad() { + try { + UtilAll.ensureDirOK(this.dbPath); + initOptions(); + + // init column family here + ColumnFamilyOptions defaultOptions = RocksDBOptionsFactory.createPopCFOptions(); + ColumnFamilyOptions popStateOptions = RocksDBOptionsFactory.createPopCFOptions(); + this.cfOptions.add(defaultOptions); + this.cfOptions.add(popStateOptions); + + List cfDescriptors = new ArrayList<>(); + cfDescriptors.add(new ColumnFamilyDescriptor(RocksDB.DEFAULT_COLUMN_FAMILY, defaultOptions)); + cfDescriptors.add(new ColumnFamilyDescriptor(COLUMN_FAMILY_NAME, popStateOptions)); + this.open(cfDescriptors); + this.defaultCFHandle = cfHandles.get(0); + this.columnFamilyHandle = cfHandles.get(1); + + log.debug("PopConsumerRocksdbStore init, filePath={}", this.dbPath); + } catch (final Exception e) { + log.error("PopConsumerRocksdbStore init error, filePath={}", this.dbPath, e); + return false; + } + return true; + } + + public String getFilePath() { + return this.dbPath; + } + + @Override + public void writeRecords(List consumerRecordList) { + if (!consumerRecordList.isEmpty()) { + try (WriteBatch writeBatch = new WriteBatch()) { + for (PopConsumerRecord record : consumerRecordList) { + writeBatch.put(columnFamilyHandle, record.getKeyBytes(), record.getValueBytes()); + } + this.db.write(writeOptions, writeBatch); + } catch (RocksDBException e) { + throw new RuntimeException("Write record error", e); + } + } + } + + @Override + public void deleteRecords(List consumerRecordList) { + if (!consumerRecordList.isEmpty()) { + try (WriteBatch writeBatch = new WriteBatch()) { + for (PopConsumerRecord record : consumerRecordList) { + writeBatch.delete(columnFamilyHandle, record.getKeyBytes()); + } + this.db.write(deleteOptions, writeBatch); + } catch (RocksDBException e) { + throw new RuntimeException("Delete record error", e); + } + } + } + + @Override + // https://github.com/facebook/rocksdb/issues/10300 + public List scanExpiredRecords(long lower, long upper, int maxCount) { + // In RocksDB, we can use SstPartitionerFixedPrefixFactory in cfOptions + // and new ColumnFamilyOptions().useFixedLengthPrefixExtractor() to + // configure prefix indexing to improve the performance of scans. + // However, in the current implementation, this is not the bottleneck. + List consumerRecordList = new ArrayList<>(); + try (ReadOptions scanOptions = new ReadOptions() + .setIterateLowerBound(new Slice(ByteBuffer.allocate(Long.BYTES).putLong(lower).array())) + .setIterateUpperBound(new Slice(ByteBuffer.allocate(Long.BYTES).putLong(upper).array())); + RocksIterator iterator = db.newIterator(this.columnFamilyHandle, scanOptions)) { + iterator.seek(ByteBuffer.allocate(Long.BYTES).putLong(lower).array()); + while (iterator.isValid() && consumerRecordList.size() < maxCount) { + consumerRecordList.add(PopConsumerRecord.decode(iterator.value())); + iterator.next(); + } + } + return consumerRecordList; + } + + @Override + protected void preShutdown() { + if (this.writeOptions != null) { + this.writeOptions.close(); + } + if (this.deleteOptions != null) { + this.deleteOptions.close(); + } + if (this.defaultCFHandle != null) { + this.defaultCFHandle.close(); + } + } +} diff --git a/broker/src/main/java/org/apache/rocketmq/broker/pop/PopConsumerService.java b/broker/src/main/java/org/apache/rocketmq/broker/pop/PopConsumerService.java new file mode 100644 index 00000000000..1f0125412a7 --- /dev/null +++ b/broker/src/main/java/org/apache/rocketmq/broker/pop/PopConsumerService.java @@ -0,0 +1,737 @@ +/* + * 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.rocketmq.broker.pop; + +import com.alibaba.fastjson.JSON; +import com.google.common.annotations.VisibleForTesting; +import com.google.common.base.Stopwatch; +import java.nio.ByteBuffer; +import java.nio.file.Paths; +import java.util.ArrayList; +import java.util.Collections; +import java.util.HashMap; +import java.util.List; +import java.util.Objects; +import java.util.Queue; +import java.util.concurrent.CompletableFuture; +import java.util.concurrent.CompletionException; +import java.util.concurrent.ConcurrentHashMap; +import java.util.concurrent.ConcurrentMap; +import java.util.concurrent.LinkedBlockingQueue; +import java.util.concurrent.TimeUnit; +import java.util.concurrent.atomic.AtomicBoolean; +import java.util.concurrent.atomic.AtomicLong; +import org.apache.commons.lang3.StringUtils; +import org.apache.commons.lang3.tuple.Triple; +import org.apache.rocketmq.broker.BrokerController; +import org.apache.rocketmq.common.BrokerConfig; +import org.apache.rocketmq.common.KeyBuilder; +import org.apache.rocketmq.common.MixAll; +import org.apache.rocketmq.common.ServiceThread; +import org.apache.rocketmq.common.TopicConfig; +import org.apache.rocketmq.common.TopicFilterType; +import org.apache.rocketmq.common.constant.LoggerName; +import org.apache.rocketmq.common.constant.PermName; +import org.apache.rocketmq.common.message.MessageAccessor; +import org.apache.rocketmq.common.message.MessageConst; +import org.apache.rocketmq.common.message.MessageDecoder; +import org.apache.rocketmq.common.message.MessageExt; +import org.apache.rocketmq.common.message.MessageExtBrokerInner; +import org.apache.rocketmq.common.utils.ConcurrentHashMapUtils; +import org.apache.rocketmq.remoting.protocol.header.ExtraInfoUtil; +import org.apache.rocketmq.store.AppendMessageStatus; +import org.apache.rocketmq.store.GetMessageResult; +import org.apache.rocketmq.store.GetMessageStatus; +import org.apache.rocketmq.store.MessageFilter; +import org.apache.rocketmq.store.PutMessageResult; +import org.apache.rocketmq.store.SelectMappedBufferResult; +import org.apache.rocketmq.store.exception.ConsumeQueueException; +import org.apache.rocketmq.store.pop.PopCheckPoint; +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; + +public class PopConsumerService extends ServiceThread { + + private static final Logger log = LoggerFactory.getLogger(LoggerName.ROCKETMQ_POP_LOGGER_NAME); + private static final long OFFSET_NOT_EXIST = -1L; + private static final String ROCKSDB_DIRECTORY = "kvStore"; + private static final int[] REWRITE_INTERVALS_IN_SECONDS = + new int[] {10, 30, 60, 120, 180, 240, 300, 360, 420, 480, 540, 600, 1200, 1800, 3600, 7200}; + + private final AtomicBoolean consumerRunning; + private final BrokerConfig brokerConfig; + private final BrokerController brokerController; + private final AtomicLong currentTime; + private final AtomicLong lastCleanupLockTime; + private final PopConsumerCache popConsumerCache; + private final PopConsumerKVStore popConsumerStore; + private final PopConsumerLockService consumerLockService; + private final ConcurrentMap requestCountTable; + + public PopConsumerService(BrokerController brokerController) { + + this.brokerController = brokerController; + this.brokerConfig = brokerController.getBrokerConfig(); + + this.consumerRunning = new AtomicBoolean(false); + this.requestCountTable = new ConcurrentHashMap<>(); + this.currentTime = new AtomicLong(TimeUnit.SECONDS.toMillis(3)); + this.lastCleanupLockTime = new AtomicLong(System.currentTimeMillis()); + this.consumerLockService = new PopConsumerLockService(TimeUnit.MINUTES.toMillis(2)); + this.popConsumerStore = new PopConsumerRocksdbStore(Paths.get( + brokerController.getMessageStoreConfig().getStorePathRootDir(), ROCKSDB_DIRECTORY).toString()); + this.popConsumerCache = brokerConfig.isEnablePopBufferMerge() ? new PopConsumerCache( + brokerController, this.popConsumerStore, this.consumerLockService, this::revive) : null; + + log.info("PopConsumerService init, buffer={}, rocksdb filePath={}", + brokerConfig.isEnablePopBufferMerge(), this.popConsumerStore.getFilePath()); + } + + /** + * In-flight messages are those that have been received from a queue + * by a consumer but have not yet been deleted. For standard queues, + * there is a limit on the number of in-flight messages, depending on queue traffic and message backlog. + */ + public boolean isPopShouldStop(String group, String topic, int queueId) { + return brokerConfig.isEnablePopMessageThreshold() && popConsumerCache != null && + popConsumerCache.getPopInFlightMessageCount(group, topic, queueId) >= + brokerConfig.getPopInflightMessageThreshold(); + } + + public long getPendingFilterCount(String groupId, String topicId, int queueId) { + try { + long maxOffset = this.brokerController.getMessageStore().getMaxOffsetInQueue(topicId, queueId); + long consumeOffset = this.brokerController.getConsumerOffsetManager().queryOffset(groupId, topicId, queueId); + return maxOffset - consumeOffset; + } catch (ConsumeQueueException e) { + throw new RuntimeException(e); + } + } + + public GetMessageResult recodeRetryMessage(GetMessageResult getMessageResult, + String topicId, long offset, long popTime, long invisibleTime) { + + if (getMessageResult.getMessageCount() == 0 || + getMessageResult.getMessageMapedList().isEmpty()) { + return getMessageResult; + } + + GetMessageResult result = new GetMessageResult(getMessageResult.getMessageCount()); + result.setStatus(GetMessageStatus.FOUND); + String brokerName = brokerConfig.getBrokerName(); + + for (SelectMappedBufferResult bufferResult : getMessageResult.getMessageMapedList()) { + List messageExtList = MessageDecoder.decodesBatch( + bufferResult.getByteBuffer(), true, false, true); + bufferResult.release(); + for (MessageExt messageExt : messageExtList) { + try { + // When override retry message topic to origin topic, + // need clear message store size to recode + String ckInfo = ExtraInfoUtil.buildExtraInfo(offset, popTime, invisibleTime, 0, + messageExt.getTopic(), brokerName, messageExt.getQueueId(), messageExt.getQueueOffset()); + messageExt.getProperties().putIfAbsent(MessageConst.PROPERTY_POP_CK, ckInfo); + messageExt.setTopic(topicId); + messageExt.setStoreSize(0); + byte[] encode = MessageDecoder.encode(messageExt, false); + ByteBuffer buffer = ByteBuffer.wrap(encode); + SelectMappedBufferResult tmpResult = new SelectMappedBufferResult( + bufferResult.getStartOffset(), buffer, encode.length, null); + result.addMessage(tmpResult); + } catch (Exception e) { + log.error("PopConsumerService exception in recode retry message, topic={}", topicId, e); + } + } + } + + return result; + } + + public PopConsumerContext addGetMessageResult(PopConsumerContext context, GetMessageResult result, + String topicId, int queueId, PopConsumerRecord.RetryType retryType, long offset) { + + if (result.getStatus() == GetMessageStatus.FOUND && !result.getMessageQueueOffset().isEmpty()) { + if (context.isFifo()) { + this.setFifoBlocked(context, context.getGroupId(), topicId, queueId, result.getMessageQueueOffset()); + } + + // build request header here + context.addGetMessageResult(result, topicId, queueId, retryType, offset); + + if (brokerConfig.isPopConsumerKVServiceLog()) { + log.info("PopConsumerService pop, time={}, invisible={}, " + + "groupId={}, topic={}, queueId={}, offset={}, attemptId={}", + context.getPopTime(), context.getInvisibleTime(), context.getGroupId(), + topicId, queueId, result.getMessageQueueOffset(), context.getAttemptId()); + } + } + + if (!context.isFifo() && result.getNextBeginOffset() > OFFSET_NOT_EXIST) { + this.brokerController.getConsumerOffsetManager().commitPullOffset( + context.getClientHost(), context.getGroupId(), topicId, queueId, result.getNextBeginOffset()); + long commitOffset = result.getStatus() == GetMessageStatus.FOUND ? offset : result.getNextBeginOffset(); + if (brokerConfig.isEnablePopBufferMerge() && popConsumerCache != null) { + long minOffset = popConsumerCache.getMinOffsetInCache(context.getGroupId(), topicId, queueId); + if (minOffset != OFFSET_NOT_EXIST) { + commitOffset = minOffset; + } + } + this.brokerController.getConsumerOffsetManager().commitOffset( + context.getClientHost(), context.getGroupId(), topicId, queueId, commitOffset); + } + + return context; + } + + public Long getPopOffset(String groupId, String topicId, int queueId) { + Long resetOffset = + this.brokerController.getConsumerOffsetManager().queryThenEraseResetOffset(topicId, groupId, queueId); + if (resetOffset != null) { + this.clearCache(groupId, topicId, queueId); + this.brokerController.getConsumerOrderInfoManager().clearBlock(topicId, groupId, queueId); + this.brokerController.getConsumerOffsetManager() + .commitOffset("ResetPopOffset", groupId, topicId, queueId, resetOffset); + } + return resetOffset; + } + + public CompletableFuture getMessageAsync(String clientHost, + String groupId, String topicId, int queueId, long offset, int batchSize, MessageFilter filter) { + + log.debug("PopConsumerService getMessageAsync, groupId={}, topicId={}, queueId={}, offset={}, batchSize={}, filter={}", + groupId, topicId, offset, queueId, batchSize, filter != null); + + Long resetOffset = this.getPopOffset(groupId, topicId, queueId); + final long currentOffset = resetOffset != null ? resetOffset : offset; + + CompletableFuture getMessageFuture = + brokerController.getMessageStore().getMessageAsync(groupId, topicId, queueId, offset, batchSize, filter); + + // refer org.apache.rocketmq.broker.processor.PopMessageProcessor#popMsgFromQueue + return getMessageFuture.thenCompose(result -> { + if (result == null) { + return CompletableFuture.completedFuture(null); + } + + // maybe store offset is not correct. + if (GetMessageStatus.OFFSET_TOO_SMALL.equals(result.getStatus()) || + GetMessageStatus.OFFSET_OVERFLOW_BADLY.equals(result.getStatus()) || + GetMessageStatus.OFFSET_FOUND_NULL.equals(result.getStatus())) { + + // commit offset, because the offset is not correct + // If offset in store is greater than cq offset, it will cause duplicate messages, + // because offset in PopBuffer is not committed. + this.brokerController.getConsumerOffsetManager().commitOffset( + clientHost, groupId, topicId, queueId, result.getNextBeginOffset()); + + log.warn("PopConsumerService getMessageAsync, initial offset because store is no correct, " + + "groupId={}, topicId={}, queueId={}, batchSize={}, offset={}->{}", + groupId, topicId, queueId, batchSize, currentOffset, result.getNextBeginOffset()); + + return brokerController.getMessageStore().getMessageAsync( + groupId, topicId, queueId, result.getNextBeginOffset(), batchSize, filter); + } + + return CompletableFuture.completedFuture(result); + + }).whenComplete((result, throwable) -> { + if (throwable != null) { + log.error("Pop getMessageAsync error", throwable); + } + }); + } + + /** + * Fifo message does not have retry feature in broker + */ + public void setFifoBlocked(PopConsumerContext context, + String groupId, String topicId, int queueId, List queueOffsetList) { + brokerController.getConsumerOrderInfoManager().update( + context.getAttemptId(), false, topicId, groupId, queueId, + context.getPopTime(), context.getInvisibleTime(), queueOffsetList, context.getOrderCountInfoBuilder()); + } + + public boolean isFifoBlocked(PopConsumerContext context, String groupId, String topicId, int queueId) { + return brokerController.getConsumerOrderInfoManager().checkBlock( + context.getAttemptId(), topicId, groupId, queueId, context.getInvisibleTime()); + } + + protected CompletableFuture getMessageAsync(CompletableFuture future, + String clientHost, String groupId, String topicId, int queueId, int batchSize, MessageFilter filter, + PopConsumerRecord.RetryType retryType) { + + return future.thenCompose(result -> { + + // pop request too much, should not add rest count here + if (isPopShouldStop(groupId, topicId, queueId)) { + return CompletableFuture.completedFuture(result); + } + + // Current requests would calculate the total number of messages + // waiting to be filtered for new message arrival notifications in + // the long-polling service, need disregarding the backlog in order + // consumption scenario. If rest message num including the blocked + // queue accumulation would lead to frequent unnecessary wake-ups + // of long-polling requests, resulting unnecessary CPU usage. + // When client ack message, long-polling request would be notifications + // by AckMessageProcessor.ackOrderly() and message will not be delayed. + if (result.isFifo() && isFifoBlocked(result, groupId, topicId, queueId)) { + // should not add accumulation(max offset - consumer offset) here + return CompletableFuture.completedFuture(result); + } + + int remain = batchSize - result.getMessageCount(); + if (remain <= 0) { + result.addRestCount(this.getPendingFilterCount(groupId, topicId, queueId)); + return CompletableFuture.completedFuture(result); + } else { + long consumeOffset = brokerController.getConsumerOffsetManager().queryPullOffset(groupId, topicId, queueId); + return getMessageAsync(clientHost, groupId, topicId, queueId, consumeOffset, remain, filter) + .thenApply(getMessageResult -> addGetMessageResult( + result, getMessageResult, topicId, queueId, retryType, consumeOffset)); + } + }); + } + + public CompletableFuture popAsync(String clientHost, long popTime, long invisibleTime, + String groupId, String topicId, int queueId, int batchSize, boolean fifo, String attemptId, + MessageFilter filter) { + + PopConsumerContext popConsumerContext = + new PopConsumerContext(clientHost, popTime, invisibleTime, groupId, fifo, attemptId); + + TopicConfig topicConfig = brokerController.getTopicConfigManager().selectTopicConfig(topicId); + if (topicConfig == null || !consumerLockService.tryLock(groupId, topicId)) { + return CompletableFuture.completedFuture(popConsumerContext); + } + + log.debug("PopConsumerService popAsync, groupId={}, topicId={}, queueId={}, " + + "batchSize={}, invisibleTime={}, fifo={}, attemptId={}, filter={}", + groupId, topicId, queueId, batchSize, invisibleTime, fifo, attemptId, filter); + + String requestKey = groupId + "@" + topicId; + String retryTopicV1 = KeyBuilder.buildPopRetryTopicV1(topicId, groupId); + String retryTopicV2 = KeyBuilder.buildPopRetryTopicV2(topicId, groupId); + long requestCount = Objects.requireNonNull(ConcurrentHashMapUtils.computeIfAbsent( + requestCountTable, requestKey, k -> new AtomicLong(0L))).getAndIncrement(); + boolean preferRetry = requestCount % 5L == 0L; + + CompletableFuture getMessageFuture = + CompletableFuture.completedFuture(popConsumerContext); + + try { + if (!fifo && preferRetry) { + if (brokerConfig.isRetrieveMessageFromPopRetryTopicV1()) { + getMessageFuture = this.getMessageAsync(getMessageFuture, clientHost, groupId, + retryTopicV1, 0, batchSize, filter, PopConsumerRecord.RetryType.RETRY_TOPIC_V1); + } + + if (brokerConfig.isEnableRetryTopicV2()) { + getMessageFuture = this.getMessageAsync(getMessageFuture, clientHost, groupId, + retryTopicV2, 0, batchSize, filter, PopConsumerRecord.RetryType.RETRY_TOPIC_V2); + } + } + + if (queueId != -1) { + getMessageFuture = this.getMessageAsync(getMessageFuture, clientHost, groupId, + topicId, queueId, batchSize, filter, PopConsumerRecord.RetryType.NORMAL_TOPIC); + } else { + for (int i = 0; i < topicConfig.getReadQueueNums(); i++) { + int current = (int) ((requestCount + i) % topicConfig.getReadQueueNums()); + getMessageFuture = this.getMessageAsync(getMessageFuture, clientHost, groupId, + topicId, current, batchSize, filter, PopConsumerRecord.RetryType.NORMAL_TOPIC); + } + + if (!fifo && !preferRetry) { + if (brokerConfig.isRetrieveMessageFromPopRetryTopicV1()) { + getMessageFuture = this.getMessageAsync(getMessageFuture, clientHost, groupId, + retryTopicV1, 0, batchSize, filter, PopConsumerRecord.RetryType.RETRY_TOPIC_V1); + } + + if (brokerConfig.isEnableRetryTopicV2()) { + getMessageFuture = this.getMessageAsync(getMessageFuture, clientHost, groupId, + retryTopicV2, 0, batchSize, filter, PopConsumerRecord.RetryType.RETRY_TOPIC_V2); + } + } + } + + return getMessageFuture.thenCompose(result -> { + if (result.isFound() && !result.isFifo()) { + if (brokerConfig.isEnablePopBufferMerge() && + popConsumerCache != null && !popConsumerCache.isCacheFull()) { + this.popConsumerCache.writeRecords(result.getPopConsumerRecordList()); + } else { + this.popConsumerStore.writeRecords(result.getPopConsumerRecordList()); + } + + for (int i = 0; i < result.getGetMessageResultList().size(); i++) { + GetMessageResult getMessageResult = result.getGetMessageResultList().get(i); + PopConsumerRecord popConsumerRecord = result.getPopConsumerRecordList().get(i); + + // If the buffer belong retries message, the message needs to be re-encoded. + // The buffer should not be re-encoded when popResponseReturnActualRetryTopic + // is true or the current topic is not a retry topic. + boolean recode = brokerConfig.isPopResponseReturnActualRetryTopic(); + if (recode && popConsumerRecord.isRetry()) { + result.getGetMessageResultList().set(i, this.recodeRetryMessage( + getMessageResult, popConsumerRecord.getTopicId(), + popConsumerRecord.getQueueId(), result.getPopTime(), invisibleTime)); + } + } + } + return CompletableFuture.completedFuture(result); + }).whenComplete((result, throwable) -> { + try { + if (throwable != null) { + log.error("PopConsumerService popAsync get message error", + throwable instanceof CompletionException ? throwable.getCause() : throwable); + } + if (result.getMessageCount() > 0) { + log.debug("PopConsumerService popAsync result, found={}, groupId={}, topicId={}, queueId={}, " + + "batchSize={}, invisibleTime={}, fifo={}, attemptId={}, filter={}", result.getMessageCount(), + groupId, topicId, queueId, batchSize, invisibleTime, fifo, attemptId, filter); + } + } finally { + consumerLockService.unlock(groupId, topicId); + } + }); + } catch (Throwable t) { + log.error("PopConsumerService popAsync error", t); + } + + return getMessageFuture; + } + + // Notify polling request when receive orderly ack + public CompletableFuture ackAsync( + long popTime, long invisibleTime, String groupId, String topicId, int queueId, long offset) { + + if (brokerConfig.isPopConsumerKVServiceLog()) { + log.info("PopConsumerService ack, time={}, invisible={}, groupId={}, topic={}, queueId={}, offset={}", + popTime, invisibleTime, groupId, topicId, queueId, offset); + } + + PopConsumerRecord record = new PopConsumerRecord( + popTime, groupId, topicId, queueId, 0, invisibleTime, offset, null); + + if (brokerConfig.isEnablePopBufferMerge() && popConsumerCache != null) { + if (popConsumerCache.deleteRecords(Collections.singletonList(record)).isEmpty()) { + return CompletableFuture.completedFuture(true); + } + } + + this.popConsumerStore.deleteRecords(Collections.singletonList(record)); + return CompletableFuture.completedFuture(true); + } + + // refer ChangeInvisibleTimeProcessor.appendCheckPointThenAckOrigin + public void changeInvisibilityDuration(long popTime, long invisibleTime, + long changedPopTime, long changedInvisibleTime, String groupId, String topicId, int queueId, long offset) { + + if (brokerConfig.isPopConsumerKVServiceLog()) { + log.info("PopConsumerService change, time={}, invisible={}, " + + "groupId={}, topic={}, queueId={}, offset={}, new time={}, new invisible={}", + popTime, invisibleTime, groupId, topicId, queueId, offset, changedPopTime, changedInvisibleTime); + } + + PopConsumerRecord ckRecord = new PopConsumerRecord( + changedPopTime, groupId, topicId, queueId, 0, changedInvisibleTime, offset, null); + + PopConsumerRecord ackRecord = new PopConsumerRecord( + popTime, groupId, topicId, queueId, 0, invisibleTime, offset, null); + + this.popConsumerStore.writeRecords(Collections.singletonList(ckRecord)); + + if (brokerConfig.isEnablePopBufferMerge() && popConsumerCache != null) { + if (popConsumerCache.deleteRecords(Collections.singletonList(ackRecord)).isEmpty()) { + return; + } + } + + this.popConsumerStore.deleteRecords(Collections.singletonList(ackRecord)); + } + + // Use broker escape bridge to support remote read + public CompletableFuture> getMessageAsync(PopConsumerRecord consumerRecord) { + return this.brokerController.getEscapeBridge().getMessageAsync(consumerRecord.getTopicId(), + consumerRecord.getOffset(), consumerRecord.getQueueId(), brokerConfig.getBrokerName(), false); + } + + public CompletableFuture revive(PopConsumerRecord record) { + return this.getMessageAsync(record) + .thenCompose(result -> { + if (result == null) { + log.error("PopConsumerService revive error, message may be lost, record={}", record); + return CompletableFuture.completedFuture(false); + } + // true in triple right means get message needs to be retried + if (result.getLeft() == null) { + log.info("PopConsumerService revive no need retry, record={}", record); + return CompletableFuture.completedFuture(!result.getRight()); + } + return CompletableFuture.completedFuture(this.reviveRetry(record, result.getLeft())); + }); + } + + public void clearCache(String groupId, String topicId, int queueId) { + while (consumerLockService.tryLock(groupId, topicId)) { + } + try { + if (popConsumerCache != null) { + popConsumerCache.removeRecords(groupId, topicId, queueId); + } + } finally { + consumerLockService.unlock(groupId, topicId); + } + } + + public long revive(AtomicLong currentTime, int maxCount) { + Stopwatch stopwatch = Stopwatch.createStarted(); + long upperTime = System.currentTimeMillis() - 50L; + List consumerRecords = this.popConsumerStore.scanExpiredRecords( + currentTime.get() - TimeUnit.SECONDS.toMillis(3), upperTime, maxCount); + long scanCostTime = stopwatch.elapsed(TimeUnit.MILLISECONDS); + Queue failureList = new LinkedBlockingQueue<>(); + List> futureList = new ArrayList<>(consumerRecords.size()); + + // could merge read operation here + for (PopConsumerRecord record : consumerRecords) { + futureList.add(this.revive(record).thenAccept(result -> { + if (!result) { + if (record.getAttemptTimes() < brokerConfig.getPopReviveMaxAttemptTimes()) { + long backoffInterval = 1000L * REWRITE_INTERVALS_IN_SECONDS[ + Math.min(REWRITE_INTERVALS_IN_SECONDS.length, record.getAttemptTimes())]; + long nextInvisibleTime = record.getInvisibleTime() + backoffInterval; + PopConsumerRecord retryRecord = new PopConsumerRecord(System.currentTimeMillis(), + record.getGroupId(), record.getTopicId(), record.getQueueId(), + record.getRetryFlag(), nextInvisibleTime, record.getOffset(), record.getAttemptId()); + retryRecord.setAttemptTimes(record.getAttemptTimes() + 1); + failureList.add(retryRecord); + log.warn("PopConsumerService revive backoff retry, record={}", retryRecord); + } else { + log.error("PopConsumerService drop record, message may be lost, record={}", record); + } + } + })); + } + + CompletableFuture.allOf(futureList.toArray(new CompletableFuture[0])).join(); + this.popConsumerStore.writeRecords(new ArrayList<>(failureList)); + this.popConsumerStore.deleteRecords(consumerRecords); + currentTime.set(consumerRecords.isEmpty() ? + upperTime : consumerRecords.get(consumerRecords.size() - 1).getVisibilityTimeout()); + + if (brokerConfig.isEnablePopBufferMerge()) { + log.info("PopConsumerService, key size={}, cache size={}, revive count={}, failure count={}, " + + "behindInMillis={}, scanInMillis={}, costInMillis={}", + popConsumerCache.getCacheKeySize(), popConsumerCache.getCacheSize(), + consumerRecords.size(), failureList.size(), upperTime - currentTime.get(), + scanCostTime, stopwatch.elapsed(TimeUnit.MILLISECONDS)); + } else { + log.info("PopConsumerService, revive count={}, failure count={}, " + + "behindInMillis={}, scanInMillis={}, costInMillis={}", + consumerRecords.size(), failureList.size(), upperTime - currentTime.get(), + scanCostTime, stopwatch.elapsed(TimeUnit.MILLISECONDS)); + } + + return consumerRecords.size(); + } + + public void createRetryTopicIfNeeded(String groupId, String topicId) { + TopicConfig topicConfig = brokerController.getTopicConfigManager().selectTopicConfig(topicId); + if (topicConfig != null) { + return; + } + + topicConfig = new TopicConfig(topicId, 1, 1, + PermName.PERM_READ | PermName.PERM_WRITE, 0); + topicConfig.setTopicFilterType(TopicFilterType.SINGLE_TAG); + brokerController.getTopicConfigManager().updateTopicConfig(topicConfig); + + long offset = this.brokerController.getConsumerOffsetManager().queryOffset(groupId, topicId, 0); + if (offset < 0) { + this.brokerController.getConsumerOffsetManager().commitOffset( + "InitPopOffset", groupId, topicId, 0, 0); + } + } + + @SuppressWarnings("DuplicatedCode") + // org.apache.rocketmq.broker.processor.PopReviveService#reviveRetry + public boolean reviveRetry(PopConsumerRecord record, MessageExt messageExt) { + + if (brokerConfig.isPopConsumerKVServiceLog()) { + log.info("PopConsumerService revive, time={}, invisible={}, groupId={}, topic={}, queueId={}, offset={}", + record.getPopTime(), record.getInvisibleTime(), record.getGroupId(), record.getTopicId(), + record.getQueueId(), record.getOffset()); + } + + boolean retry = StringUtils.startsWith(record.getTopicId(), MixAll.RETRY_GROUP_TOPIC_PREFIX); + String retryTopic = retry ? record.getTopicId() : KeyBuilder.buildPopRetryTopic( + record.getTopicId(), record.getGroupId(), brokerConfig.isEnableRetryTopicV2()); + this.createRetryTopicIfNeeded(record.getGroupId(), retryTopic); + + // deep copy here + MessageExtBrokerInner msgInner = new MessageExtBrokerInner(); + msgInner.setTopic(retryTopic); + msgInner.setBody(messageExt.getBody() != null ? messageExt.getBody() : new byte[] {}); + msgInner.setQueueId(0); + if (messageExt.getTags() != null) { + msgInner.setTags(messageExt.getTags()); + } else { + MessageAccessor.setProperties(msgInner, new HashMap<>()); + } + + msgInner.setBornTimestamp(messageExt.getBornTimestamp()); + msgInner.setFlag(messageExt.getFlag()); + msgInner.setSysFlag(messageExt.getSysFlag()); + msgInner.setBornHost(brokerController.getStoreHost()); + msgInner.setStoreHost(brokerController.getStoreHost()); + msgInner.setReconsumeTimes(messageExt.getReconsumeTimes() + 1); + msgInner.getProperties().putAll(messageExt.getProperties()); + + // set first pop time here + if (messageExt.getReconsumeTimes() == 0 || + msgInner.getProperties().get(MessageConst.PROPERTY_FIRST_POP_TIME) == null) { + msgInner.getProperties().put(MessageConst.PROPERTY_FIRST_POP_TIME, String.valueOf(record.getPopTime())); + } + msgInner.setPropertiesString(MessageDecoder.messageProperties2String(msgInner.getProperties())); + + PutMessageResult putMessageResult = + brokerController.getEscapeBridge().putMessageToSpecificQueue(msgInner); + + if (putMessageResult.getAppendMessageResult() == null || + putMessageResult.getAppendMessageResult().getStatus() != AppendMessageStatus.PUT_OK) { + log.error("PopConsumerService revive retry msg error, put status={}, ck={}, delay={}ms", + putMessageResult, JSON.toJSONString(record), System.currentTimeMillis() - record.getVisibilityTimeout()); + return false; + } + + if (this.brokerController.getBrokerStatsManager() != null) { + this.brokerController.getBrokerStatsManager().incBrokerPutNums(msgInner.getTopic(), 1); + this.brokerController.getBrokerStatsManager().incTopicPutNums(msgInner.getTopic()); + this.brokerController.getBrokerStatsManager().incTopicPutSize( + msgInner.getTopic(), putMessageResult.getAppendMessageResult().getWroteBytes()); + } + return true; + } + + // Export kv store record to revive topic + @SuppressWarnings("ExtractMethodRecommender") + public synchronized void transferToFsStore() { + Stopwatch stopwatch = Stopwatch.createStarted(); + while (true) { + try { + List consumerRecords = this.popConsumerStore.scanExpiredRecords( + 0, Long.MAX_VALUE, brokerConfig.getPopReviveMaxReturnSizePerRead()); + if (consumerRecords == null || consumerRecords.isEmpty()) { + break; + } + for (PopConsumerRecord record : consumerRecords) { + PopCheckPoint ck = new PopCheckPoint(); + ck.setBitMap(0); + ck.setNum((byte) 1); + ck.setPopTime(record.getPopTime()); + ck.setInvisibleTime(record.getInvisibleTime()); + ck.setStartOffset(record.getOffset()); + ck.setCId(record.getGroupId()); + ck.setTopic(record.getTopicId()); + ck.setQueueId(record.getQueueId()); + ck.setBrokerName(brokerConfig.getBrokerName()); + ck.addDiff(0); + ck.setRePutTimes(ck.getRePutTimes()); + int reviveQueueId = (int) record.getOffset() % brokerConfig.getReviveQueueNum(); + MessageExtBrokerInner ckMsg = + brokerController.getPopMessageProcessor().buildCkMsg(ck, reviveQueueId); + brokerController.getMessageStore().asyncPutMessage(ckMsg).join(); + } + log.info("PopConsumerStore transfer from kvStore to fsStore, count={}", consumerRecords.size()); + this.popConsumerStore.deleteRecords(consumerRecords); + this.waitForRunning(1); + } catch (Throwable t) { + log.error("PopConsumerStore transfer from kvStore to fsStore failure", t); + } + } + log.info("PopConsumerStore transfer to fsStore finish, cost={}ms", stopwatch.elapsed(TimeUnit.MILLISECONDS)); + } + + @Override + public String getServiceName() { + return PopConsumerService.class.getSimpleName(); + } + + @VisibleForTesting + protected PopConsumerKVStore getPopConsumerStore() { + return popConsumerStore; + } + + public PopConsumerLockService getConsumerLockService() { + return consumerLockService; + } + + @Override + public void start() { + if (!this.popConsumerStore.start()) { + throw new RuntimeException("PopConsumerStore init error"); + } + if (this.popConsumerCache != null) { + this.popConsumerCache.start(); + } + super.start(); + } + + @Override + public void shutdown() { + // Block shutdown thread until write records finish + super.shutdown(); + do { + this.waitForRunning(10); + } + while (consumerRunning.get()); + if (this.popConsumerCache != null) { + this.popConsumerCache.shutdown(); + } + if (this.popConsumerStore != null) { + this.popConsumerStore.shutdown(); + } + } + + @Override + public void run() { + this.consumerRunning.set(true); + while (!isStopped()) { + try { + // to prevent concurrency issues during read and write operations + long reviveCount = this.revive(this.currentTime, + brokerConfig.getPopReviveMaxReturnSizePerRead()); + + long current = System.currentTimeMillis(); + if (lastCleanupLockTime.get() + TimeUnit.MINUTES.toMillis(1) < current) { + this.consumerLockService.removeTimeout(); + this.lastCleanupLockTime.set(current); + } + + if (reviveCount < brokerConfig.getPopReviveMaxReturnSizePerRead()) { + this.waitForRunning(500); + } + } catch (Exception e) { + log.error("PopConsumerService revive error", e); + this.waitForRunning(500); + } + } + this.consumerRunning.set(false); + } +} diff --git a/broker/src/main/java/org/apache/rocketmq/broker/processor/AckMessageProcessor.java b/broker/src/main/java/org/apache/rocketmq/broker/processor/AckMessageProcessor.java index 043ef13f5a9..23a4f6167c6 100644 --- a/broker/src/main/java/org/apache/rocketmq/broker/processor/AckMessageProcessor.java +++ b/broker/src/main/java/org/apache/rocketmq/broker/processor/AckMessageProcessor.java @@ -23,6 +23,9 @@ import java.nio.charset.StandardCharsets; import org.apache.rocketmq.broker.BrokerController; import org.apache.rocketmq.broker.metrics.PopMetricsManager; +import org.apache.rocketmq.broker.offset.ConsumerOffsetManager; +import org.apache.rocketmq.broker.offset.ConsumerOrderInfoManager; +import org.apache.rocketmq.broker.pop.PopConsumerLockService; import org.apache.rocketmq.common.KeyBuilder; import org.apache.rocketmq.common.PopAckConstants; import org.apache.rocketmq.common.TopicConfig; @@ -50,6 +53,7 @@ import org.apache.rocketmq.store.pop.BatchAckMsg; public class AckMessageProcessor implements NettyRequestProcessor { + private static final Logger POP_LOGGER = LoggerFactory.getLogger(LoggerName.ROCKETMQ_POP_LOGGER_NAME); private final BrokerController brokerController; private final String reviveTopic; @@ -57,7 +61,8 @@ public class AckMessageProcessor implements NettyRequestProcessor { public AckMessageProcessor(final BrokerController brokerController) { this.brokerController = brokerController; - this.reviveTopic = PopAckConstants.buildClusterReviveTopic(this.brokerController.getBrokerConfig().getBrokerClusterName()); + this.reviveTopic = PopAckConstants.buildClusterReviveTopic( + this.brokerController.getBrokerConfig().getBrokerClusterName()); this.popReviveServices = new PopReviveService[this.brokerController.getBrokerConfig().getReviveQueueNum()]; for (int i = 0; i < this.brokerController.getBrokerConfig().getReviveQueueNum(); i++) { this.popReviveServices[i] = new PopReviveService(brokerController, reviveTopic, i); @@ -149,8 +154,11 @@ private RemotingCommand processRequest(final Channel channel, RemotingCommand re response.setRemark(errorInfo); return response; } - - appendAck(requestHeader, null, response, channel, null); + if (brokerController.getBrokerConfig().isPopConsumerKVServiceEnable()) { + appendAckNew(requestHeader, null, response, channel, null); + } else { + appendAck(requestHeader, null, response, channel, null); + } } else if (request.getCode() == RequestCode.BATCH_ACK_MESSAGE) { if (request.getBody() != null) { reqBody = BatchAckMessageRequestBody.decode(request.getBody(), BatchAckMessageRequestBody.class); @@ -160,7 +168,11 @@ private RemotingCommand processRequest(final Channel channel, RemotingCommand re return response; } for (BatchAck bAck : reqBody.getAcks()) { - appendAck(null, bAck, response, channel, reqBody.getBrokerName()); + if (brokerController.getBrokerConfig().isPopConsumerKVServiceEnable()) { + appendAckNew(null, bAck, response, channel, reqBody.getBrokerName()); + } else { + appendAck(null, bAck, response, channel, reqBody.getBrokerName()); + } } } else { POP_LOGGER.error("AckMessageProcessor failed to process RequestCode: {}, consumer: {} ", request.getCode(), RemotingHelper.parseChannelRemoteAddr(channel)); @@ -296,6 +308,74 @@ private void appendAck(final AckMessageRequestHeader requestHeader, final BatchA } } + private void appendAckNew(final AckMessageRequestHeader requestHeader, final BatchAck batchAck, + final RemotingCommand response, final Channel channel, String brokerName) throws RemotingCommandException { + + if (requestHeader != null && batchAck == null) { + String[] extraInfo = ExtraInfoUtil.split(requestHeader.getExtraInfo()); + String groupId = requestHeader.getConsumerGroup(); + String topicId = requestHeader.getTopic(); + int queueId = requestHeader.getQueueId(); + long ackOffset = requestHeader.getOffset(); + long popTime = ExtraInfoUtil.getPopTime(extraInfo); + long invisibleTime = ExtraInfoUtil.getInvisibleTime(extraInfo); + + int reviveQueueId = ExtraInfoUtil.getReviveQid(extraInfo); + if (reviveQueueId == KeyBuilder.POP_ORDER_REVIVE_QUEUE) { + ackOrderlyNew(topicId, groupId, queueId, ackOffset, popTime, invisibleTime, channel, response); + } else { + this.brokerController.getPopConsumerService().ackAsync( + popTime, invisibleTime, groupId, topicId, queueId, ackOffset); + } + + this.brokerController.getBrokerStatsManager().incBrokerAckNums(1); + this.brokerController.getBrokerStatsManager().incGroupAckNums(groupId, topicId, 1); + } else { + String groupId = batchAck.getConsumerGroup(); + String topicId = ExtraInfoUtil.getRealTopic( + batchAck.getTopic(), batchAck.getConsumerGroup(), batchAck.getRetry()); + int queueId = batchAck.getQueueId(); + int reviveQueueId = batchAck.getReviveQueueId(); + long startOffset = batchAck.getStartOffset(); + long popTime = batchAck.getPopTime(); + long invisibleTime = batchAck.getInvisibleTime(); + + try { + long minOffset = this.brokerController.getMessageStore().getMinOffsetInQueue(topicId, queueId); + long maxOffset = this.brokerController.getMessageStore().getMaxOffsetInQueue(topicId, queueId); + if (minOffset == -1 || maxOffset == -1) { + POP_LOGGER.error("Illegal topic or queue found when batch ack {}", batchAck); + return; + } + + int ackCount = 0; + // Maintain consistency with the old implementation code style + BitSet bitSet = batchAck.getBitSet(); + for (int i = bitSet.nextSetBit(0); i >= 0; i = bitSet.nextSetBit(i + 1)) { + if (i == Integer.MAX_VALUE) { + break; + } + long offset = startOffset + i; + if (offset < minOffset || offset > maxOffset) { + continue; + } + if (reviveQueueId == KeyBuilder.POP_ORDER_REVIVE_QUEUE) { + ackOrderlyNew(topicId, groupId, queueId, offset, popTime, invisibleTime, channel, response); + } else { + this.brokerController.getPopConsumerService().ackAsync( + popTime, invisibleTime, groupId, topicId, queueId, offset); + } + ackCount++; + } + + this.brokerController.getBrokerStatsManager().incBrokerAckNums(ackCount); + this.brokerController.getBrokerStatsManager().incGroupAckNums(groupId, topicId, ackCount); + } catch (ConsumeQueueException e) { + throw new RemotingCommandException("Failed to ack message", e); + } + } + } + private void handlePutMessageResult(PutMessageResult putMessageResult, AckMsg ackMsg, String topic, String consumeGroup, long popTime, int qId, int ackCount) { if (putMessageResult.getPutMessageStatus() != PutMessageStatus.PUT_OK @@ -323,9 +403,7 @@ protected void ackOrderly(String topic, String consumeGroup, int qId, long ackOf return; } long nextOffset = brokerController.getConsumerOrderInfoManager().commitAndNext( - topic, consumeGroup, - qId, ackOffset, - popTime); + topic, consumeGroup, qId, ackOffset, popTime); if (nextOffset > -1) { if (!this.brokerController.getConsumerOffsetManager().hasOffsetReset(topic, consumeGroup, qId)) { this.brokerController.getConsumerOffsetManager().commitOffset( @@ -347,4 +425,55 @@ protected void ackOrderly(String topic, String consumeGroup, int qId, long ackOf } brokerController.getPopInflightMessageCounter().decrementInFlightMessageNum(topic, consumeGroup, popTime, qId, 1); } + + protected void ackOrderlyNew(String topic, String consumeGroup, int qId, long ackOffset, long popTime, + long invisibleTime, Channel channel, RemotingCommand response) { + + ConsumerOffsetManager consumerOffsetManager = this.brokerController.getConsumerOffsetManager(); + ConsumerOrderInfoManager consumerOrderInfoManager = brokerController.getConsumerOrderInfoManager(); + PopConsumerLockService consumerLockService = this.brokerController.getPopConsumerService().getConsumerLockService(); + + long oldOffset = consumerOffsetManager.queryOffset(consumeGroup, topic, qId); + if (ackOffset < oldOffset) { + return; + } + + while (!consumerLockService.tryLock(consumeGroup, topic)) { + } + + try { + // double check + oldOffset = consumerOffsetManager.queryOffset(consumeGroup, topic, qId); + if (ackOffset < oldOffset) { + return; + } + + long nextOffset = consumerOrderInfoManager.commitAndNext(topic, consumeGroup, qId, ackOffset, popTime); + if (brokerController.getBrokerConfig().isPopConsumerKVServiceLog()) { + POP_LOGGER.info("PopConsumerService ack orderly, time={}, topicId={}, groupId={}, queueId={}, " + + "offset={}, next={}", popTime, topic, consumeGroup, qId, ackOffset, nextOffset); + } + + if (nextOffset > -1L) { + if (!consumerOffsetManager.hasOffsetReset(topic, consumeGroup, qId)) { + String remoteAddress = RemotingHelper.parseSocketAddressAddr(channel.remoteAddress()); + consumerOffsetManager.commitOffset(remoteAddress, consumeGroup, topic, qId, nextOffset); + } + if (!consumerOrderInfoManager.checkBlock(null, topic, consumeGroup, qId, invisibleTime)) { + this.brokerController.getPopMessageProcessor().notifyMessageArriving(topic, qId, consumeGroup); + } + return; + } + + if (nextOffset == -1) { + String errorInfo = String.format("offset is illegal, key:%s %s %s, old:%d, commit:%d, next:%d, %s", + consumeGroup, topic, qId, oldOffset, ackOffset, nextOffset, channel.remoteAddress()); + POP_LOGGER.warn(errorInfo); + response.setCode(ResponseCode.MESSAGE_ILLEGAL); + response.setRemark(errorInfo); + } + } finally { + consumerLockService.unlock(consumeGroup, topic); + } + } } diff --git a/broker/src/main/java/org/apache/rocketmq/broker/processor/AdminBrokerProcessor.java b/broker/src/main/java/org/apache/rocketmq/broker/processor/AdminBrokerProcessor.java index b16fcc9ddf7..c5a3051a5ff 100644 --- a/broker/src/main/java/org/apache/rocketmq/broker/processor/AdminBrokerProcessor.java +++ b/broker/src/main/java/org/apache/rocketmq/broker/processor/AdminBrokerProcessor.java @@ -35,6 +35,7 @@ import java.util.Properties; import java.util.Set; import java.util.concurrent.CompletableFuture; +import java.util.concurrent.CompletionException; import java.util.concurrent.ConcurrentHashMap; import java.util.concurrent.ConcurrentMap; import java.util.concurrent.CountDownLatch; @@ -60,6 +61,9 @@ import org.apache.rocketmq.broker.auth.converter.UserConverter; import org.apache.rocketmq.broker.client.ClientChannelInfo; import org.apache.rocketmq.broker.client.ConsumerGroupInfo; +import org.apache.rocketmq.broker.config.v1.RocksDBConsumerOffsetManager; +import org.apache.rocketmq.broker.config.v1.RocksDBSubscriptionGroupManager; +import org.apache.rocketmq.broker.config.v1.RocksDBTopicConfigManager; import org.apache.rocketmq.broker.controller.ReplicasManager; import org.apache.rocketmq.broker.filter.ConsumerFilterData; import org.apache.rocketmq.broker.filter.ExpressionMessageFilter; @@ -159,6 +163,7 @@ import org.apache.rocketmq.remoting.protocol.header.DeleteUserRequestHeader; import org.apache.rocketmq.remoting.protocol.header.ExchangeHAInfoRequestHeader; import org.apache.rocketmq.remoting.protocol.header.ExchangeHAInfoResponseHeader; +import org.apache.rocketmq.remoting.protocol.header.ExportRocksDBConfigToJsonRequestHeader; import org.apache.rocketmq.remoting.protocol.header.GetAclRequestHeader; import org.apache.rocketmq.remoting.protocol.header.GetAllProducerInfoRequestHeader; import org.apache.rocketmq.remoting.protocol.header.GetAllTopicConfigResponseHeader; @@ -239,7 +244,7 @@ public class AdminBrokerProcessor implements NettyRequestProcessor { private static final Logger LOGGER = LoggerFactory.getLogger(LoggerName.BROKER_LOGGER_NAME); protected final BrokerController brokerController; protected Set configBlackList = new HashSet<>(); - private final ExecutorService asyncExecuteWorker = new ThreadPoolExecutor(0, 1, 60L, TimeUnit.SECONDS, new SynchronousQueue<>()); + private final ExecutorService asyncExecuteWorker = new ThreadPoolExecutor(0, 4, 60L, TimeUnit.SECONDS, new SynchronousQueue<>()); public AdminBrokerProcessor(final BrokerController brokerController) { this.brokerController = brokerController; @@ -356,6 +361,8 @@ public RemotingCommand processRequest(ChannelHandlerContext ctx, return queryConsumeQueue(ctx, request); case RequestCode.CHECK_ROCKSDB_CQ_WRITE_PROGRESS: return this.checkRocksdbCqWriteProgress(ctx, request); + case RequestCode.EXPORT_ROCKSDB_CONFIG_TO_JSON: + return this.exportRocksDBConfigToJson(ctx, request); case RequestCode.UPDATE_AND_GET_GROUP_FORBIDDEN: return this.updateAndGetGroupForbidden(ctx, request); case RequestCode.GET_SUBSCRIPTIONGROUP_CONFIG: @@ -406,6 +413,8 @@ public RemotingCommand processRequest(ChannelHandlerContext ctx, return this.getAcl(ctx, request); case RequestCode.AUTH_LIST_ACL: return this.listAcl(ctx, request); + case RequestCode.POP_ROLLBACK: + return this.transferPopToFsStore(ctx, request); default: return getUnknownCmdResponse(ctx, request); } @@ -422,7 +431,7 @@ private RemotingCommand getSubscriptionGroup(ChannelHandlerContext ctx, GetSubscriptionGroupConfigRequestHeader requestHeader = (GetSubscriptionGroupConfigRequestHeader) request.decodeCommandCustomHeader(GetSubscriptionGroupConfigRequestHeader.class); final RemotingCommand response = RemotingCommand.createResponseCommand(null); - SubscriptionGroupConfig groupConfig = this.brokerController.getSubscriptionGroupManager().getSubscriptionGroupTable().get(requestHeader.getGroup()); + SubscriptionGroupConfig groupConfig = this.brokerController.getSubscriptionGroupManager().findSubscriptionGroupConfig(requestHeader.getGroup()); if (groupConfig == null) { LOGGER.error("No group in this broker, client: {} group: {}", ctx.channel().remoteAddress(), requestHeader.getGroup()); response.setCode(ResponseCode.SUBSCRIPTION_GROUP_NOT_EXIST); @@ -493,6 +502,51 @@ private RemotingCommand checkRocksdbCqWriteProgress(ChannelHandlerContext ctx, R return response; } + private RemotingCommand exportRocksDBConfigToJson(ChannelHandlerContext ctx, + RemotingCommand request) throws RemotingCommandException { + ExportRocksDBConfigToJsonRequestHeader requestHeader = request.decodeCommandCustomHeader(ExportRocksDBConfigToJsonRequestHeader.class); + List configTypes = requestHeader.fetchConfigType(); + List> futureList = new ArrayList<>(configTypes.size()); + for (ExportRocksDBConfigToJsonRequestHeader.ConfigType type : configTypes) { + switch (type) { + case TOPICS: + if (this.brokerController.getTopicConfigManager() instanceof RocksDBTopicConfigManager) { + RocksDBTopicConfigManager rocksDBTopicConfigManager = (RocksDBTopicConfigManager) this.brokerController.getTopicConfigManager(); + futureList.add(CompletableFuture.runAsync(rocksDBTopicConfigManager::exportToJson, asyncExecuteWorker)); + } + break; + case SUBSCRIPTION_GROUPS: + if (this.brokerController.getSubscriptionGroupManager() instanceof RocksDBSubscriptionGroupManager) { + RocksDBSubscriptionGroupManager rocksDBSubscriptionGroupManager = (RocksDBSubscriptionGroupManager) this.brokerController.getSubscriptionGroupManager(); + futureList.add(CompletableFuture.runAsync(rocksDBSubscriptionGroupManager::exportToJson, asyncExecuteWorker)); + } + break; + case CONSUMER_OFFSETS: + if (this.brokerController.getConsumerOffsetManager() instanceof RocksDBConsumerOffsetManager) { + RocksDBConsumerOffsetManager rocksDBConsumerOffsetManager = (RocksDBConsumerOffsetManager) this.brokerController.getConsumerOffsetManager(); + futureList.add(CompletableFuture.runAsync(rocksDBConsumerOffsetManager::exportToJson, asyncExecuteWorker)); + } + break; + default: + break; + } + } + + try { + CompletableFuture.allOf(futureList.toArray(new CompletableFuture[0])).join(); + } catch (CompletionException e) { + RemotingCommand response = RemotingCommand.createResponseCommand(null); + response.setCode(ResponseCode.SYSTEM_ERROR); + response.setRemark(String.valueOf(e)); + return response; + } + + RemotingCommand response = RemotingCommand.createResponseCommand(null); + response.setCode(ResponseCode.SUCCESS); + response.setRemark("export done."); + return response; + } + @Override public boolean rejectRequest() { return false; @@ -2200,7 +2254,14 @@ private RemotingCommand resetOffsetInner(String topic, String group, int queueId ResetOffsetBody body = new ResetOffsetBody(); String brokerName = brokerController.getBrokerConfig().getBrokerName(); for (Map.Entry entry : queueOffsetMap.entrySet()) { - brokerController.getPopInflightMessageCounter().clearInFlightMessageNum(topic, group, entry.getKey()); + if (brokerController.getPopInflightMessageCounter() != null) { + brokerController.getPopInflightMessageCounter().clearInFlightMessageNum(topic, group, entry.getKey()); + } + if (brokerController.getBrokerConfig().isPopConsumerKVServiceEnable()) { + brokerController.getPopConsumerService().clearCache(group, topic, entry.getKey()); + brokerController.getConsumerOffsetManager().commitPullOffset( + "ResetOffsetInner", group, topic, entry.getKey(), entry.getValue()); + } body.getOffsetTable().put(new MessageQueue(topic, brokerName, entry.getKey()), entry.getValue()); } @@ -2450,7 +2511,7 @@ private RemotingCommand consumeMessageDirectly(ChannelHandlerContext ctx, } // groupSysFlag if (StringUtils.isNotEmpty(requestHeader.getConsumerGroup())) { - SubscriptionGroupConfig groupConfig = brokerController.getSubscriptionGroupManager().getSubscriptionGroupTable().get(requestHeader.getConsumerGroup()); + SubscriptionGroupConfig groupConfig = brokerController.getSubscriptionGroupManager().findSubscriptionGroupConfig(requestHeader.getConsumerGroup()); if (groupConfig != null) { request.addExtField("groupSysFlag", String.valueOf(groupConfig.getGroupSysFlag())); } @@ -2939,7 +3000,7 @@ private RemotingCommand getTopicConfig(ChannelHandlerContext ctx, GetTopicConfigRequestHeader requestHeader = (GetTopicConfigRequestHeader) request.decodeCommandCustomHeader(GetTopicConfigRequestHeader.class); final RemotingCommand response = RemotingCommand.createResponseCommand(null); - TopicConfig topicConfig = this.brokerController.getTopicConfigManager().getTopicConfigTable().get(requestHeader.getTopic()); + TopicConfig topicConfig = this.brokerController.getTopicConfigManager().selectTopicConfig(requestHeader.getTopic()); if (topicConfig == null) { LOGGER.error("No topic in this broker, client: {} topic: {}", ctx.channel().remoteAddress(), requestHeader.getTopic()); //be care of the response code, should set "not-exist" explicitly @@ -3536,4 +3597,19 @@ private boolean checkCqUnitEqual(CqUnit cqUnit1, CqUnit cqUnit2) { } return cqUnit1.getTagsCode() == cqUnit2.getTagsCode(); } + + private RemotingCommand transferPopToFsStore(ChannelHandlerContext ctx, RemotingCommand request) { + final RemotingCommand response = RemotingCommand.createResponseCommand(null); + try { + if (brokerController.getPopConsumerService() != null) { + brokerController.getPopConsumerService().transferToFsStore(); + } + response.setCode(ResponseCode.SUCCESS); + } catch (Exception e) { + LOGGER.error("PopConsumerStore transfer from kvStore to fsStore finish [{}]", request, e); + response.setCode(ResponseCode.SYSTEM_ERROR); + response.setRemark(e.getMessage()); + } + return response; + } } diff --git a/broker/src/main/java/org/apache/rocketmq/broker/processor/ChangeInvisibleTimeProcessor.java b/broker/src/main/java/org/apache/rocketmq/broker/processor/ChangeInvisibleTimeProcessor.java index d29ff2a55b0..a7180f66545 100644 --- a/broker/src/main/java/org/apache/rocketmq/broker/processor/ChangeInvisibleTimeProcessor.java +++ b/broker/src/main/java/org/apache/rocketmq/broker/processor/ChangeInvisibleTimeProcessor.java @@ -24,6 +24,9 @@ import java.nio.charset.StandardCharsets; import org.apache.rocketmq.broker.BrokerController; import org.apache.rocketmq.broker.metrics.PopMetricsManager; +import org.apache.rocketmq.broker.offset.ConsumerOffsetManager; +import org.apache.rocketmq.broker.offset.ConsumerOrderInfoManager; +import org.apache.rocketmq.broker.pop.PopConsumerLockService; import org.apache.rocketmq.common.PopAckConstants; import org.apache.rocketmq.common.TopicConfig; import org.apache.rocketmq.common.constant.LoggerName; @@ -133,15 +136,36 @@ public CompletableFuture processRequestAsync(final Channel chan } String[] extraInfo = ExtraInfoUtil.split(requestHeader.getExtraInfo()); + if (brokerController.getBrokerConfig().isPopConsumerKVServiceEnable()) { + if (ExtraInfoUtil.isOrder(extraInfo)) { + return this.processChangeInvisibleTimeForOrderNew( + requestHeader, extraInfo, response, responseHeader); + } + try { + long current = System.currentTimeMillis(); + brokerController.getPopConsumerService().changeInvisibilityDuration( + ExtraInfoUtil.getPopTime(extraInfo), ExtraInfoUtil.getInvisibleTime(extraInfo), current, + requestHeader.getInvisibleTime(), requestHeader.getConsumerGroup(), requestHeader.getTopic(), + requestHeader.getQueueId(), requestHeader.getOffset()); + responseHeader.setInvisibleTime(requestHeader.getInvisibleTime()); + responseHeader.setPopTime(current); + responseHeader.setReviveQid(ExtraInfoUtil.getReviveQid(extraInfo)); + } catch (Exception e) { + response.setCode(ResponseCode.SYSTEM_ERROR); + } + return CompletableFuture.completedFuture(response); + } if (ExtraInfoUtil.isOrder(extraInfo)) { - return CompletableFuture.completedFuture(processChangeInvisibleTimeForOrder(requestHeader, extraInfo, response, responseHeader)); + return CompletableFuture.completedFuture( + processChangeInvisibleTimeForOrder(requestHeader, extraInfo, response, responseHeader)); } // add new ck long now = System.currentTimeMillis(); + CompletableFuture futureResult = appendCheckPointThenAckOrigin(requestHeader, + ExtraInfoUtil.getReviveQid(extraInfo), requestHeader.getQueueId(), requestHeader.getOffset(), now, extraInfo); - CompletableFuture futureResult = appendCheckPointThenAckOrigin(requestHeader, ExtraInfoUtil.getReviveQid(extraInfo), requestHeader.getQueueId(), requestHeader.getOffset(), now, extraInfo); return futureResult.thenCompose(result -> { if (result) { responseHeader.setInvisibleTime(requestHeader.getInvisibleTime()); @@ -154,6 +178,50 @@ public CompletableFuture processRequestAsync(final Channel chan }); } + @SuppressWarnings({"StatementWithEmptyBody", "DuplicatedCode"}) + public CompletableFuture processChangeInvisibleTimeForOrderNew( + ChangeInvisibleTimeRequestHeader requestHeader, String[] extraInfo, + RemotingCommand response, ChangeInvisibleTimeResponseHeader responseHeader) { + + String groupId = requestHeader.getConsumerGroup(); + String topicId = requestHeader.getTopic(); + Integer queueId = requestHeader.getQueueId(); + long popTime = ExtraInfoUtil.getPopTime(extraInfo); + + PopConsumerLockService consumerLockService = + this.brokerController.getPopConsumerService().getConsumerLockService(); + ConsumerOffsetManager consumerOffsetManager = this.brokerController.getConsumerOffsetManager(); + ConsumerOrderInfoManager consumerOrderInfoManager = brokerController.getConsumerOrderInfoManager(); + + long oldOffset = consumerOffsetManager.queryOffset(groupId, topicId, queueId); + if (requestHeader.getOffset() < oldOffset) { + return CompletableFuture.completedFuture(response); + } + + while (!consumerLockService.tryLock(groupId, topicId)) { + } + + try { + // double check + oldOffset = consumerOffsetManager.queryOffset(groupId, topicId, queueId); + if (requestHeader.getOffset() < oldOffset) { + return CompletableFuture.completedFuture(response); + } + + long visibilityTimeout = System.currentTimeMillis() + requestHeader.getInvisibleTime(); + consumerOrderInfoManager.updateNextVisibleTime( + topicId, groupId, queueId, requestHeader.getOffset(), popTime, visibilityTimeout); + + responseHeader.setInvisibleTime(visibilityTimeout - popTime); + responseHeader.setPopTime(popTime); + responseHeader.setReviveQid(ExtraInfoUtil.getReviveQid(extraInfo)); + } finally { + consumerLockService.unlock(groupId, topicId); + } + + return CompletableFuture.completedFuture(response); + } + protected RemotingCommand processChangeInvisibleTimeForOrder(ChangeInvisibleTimeRequestHeader requestHeader, String[] extraInfo, RemotingCommand response, ChangeInvisibleTimeResponseHeader responseHeader) { long popTime = ExtraInfoUtil.getPopTime(extraInfo); diff --git a/broker/src/main/java/org/apache/rocketmq/broker/processor/NotificationProcessor.java b/broker/src/main/java/org/apache/rocketmq/broker/processor/NotificationProcessor.java index 6317d6ad7d2..b95055efba7 100644 --- a/broker/src/main/java/org/apache/rocketmq/broker/processor/NotificationProcessor.java +++ b/broker/src/main/java/org/apache/rocketmq/broker/processor/NotificationProcessor.java @@ -172,7 +172,6 @@ public RemotingCommand processRequest(final ChannelHandlerContext ctx, private boolean hasMsgFromTopic(String topicName, int randomQ, NotificationRequestHeader requestHeader) throws RemotingCommandException { - boolean hasMsg; TopicConfig topicConfig = this.brokerController.getTopicConfigManager().selectTopicConfig(topicName); return hasMsgFromTopic(topicConfig, randomQ, requestHeader); } @@ -212,13 +211,16 @@ private long getPopOffset(String topic, String cid, int queueId) { if (offset < 0) { offset = this.brokerController.getMessageStore().getMinOffsetInQueue(topic, queueId); } - long bufferOffset = this.brokerController.getPopMessageProcessor().getPopBufferMergeService() - .getLatestOffset(topic, cid, queueId); - if (bufferOffset < 0) { - return offset; + + long bufferOffset; + if (brokerController.getBrokerConfig().isPopConsumerKVServiceEnable()) { + bufferOffset = this.brokerController.getConsumerOffsetManager().queryPullOffset(cid, topic, queueId); } else { - return bufferOffset > offset ? bufferOffset : offset; + bufferOffset = this.brokerController.getPopMessageProcessor() + .getPopBufferMergeService().getLatestOffset(topic, cid, queueId); } + + return bufferOffset < 0L ? offset : Math.max(bufferOffset, offset); } public PopLongPollingService getPopLongPollingService() { diff --git a/broker/src/main/java/org/apache/rocketmq/broker/processor/PopBufferMergeService.java b/broker/src/main/java/org/apache/rocketmq/broker/processor/PopBufferMergeService.java index 9f10b483ddb..820388b18d2 100644 --- a/broker/src/main/java/org/apache/rocketmq/broker/processor/PopBufferMergeService.java +++ b/broker/src/main/java/org/apache/rocketmq/broker/processor/PopBufferMergeService.java @@ -62,7 +62,7 @@ public class PopBufferMergeService extends ServiceThread { private final int countOfSecond1 = (int) (1000 / interval); private final int countOfSecond30 = (int) (30 * 1000 / interval); - private final List batchAckIndexList = new ArrayList(32); + private final List batchAckIndexList = new ArrayList<>(32); private volatile boolean master = false; public PopBufferMergeService(BrokerController brokerController, PopMessageProcessor popMessageProcessor) { @@ -197,12 +197,12 @@ private void scanGarbage() { String topic = keyArray[0]; String cid = keyArray[1]; if (brokerController.getTopicConfigManager().selectTopicConfig(topic) == null) { - POP_LOGGER.info("[PopBuffer]remove not exit topic {} in buffer!", topic); + POP_LOGGER.info("[PopBuffer]remove nonexistent topic {} in buffer!", topic); iterator.remove(); continue; } - if (!brokerController.getSubscriptionGroupManager().getSubscriptionGroupTable().containsKey(cid)) { - POP_LOGGER.info("[PopBuffer]remove not exit sub {} of topic {} in buffer!", cid, topic); + if (!brokerController.getSubscriptionGroupManager().containsSubscriptionGroup(cid)) { + POP_LOGGER.info("[PopBuffer]remove nonexistent subscription group {} of topic {} in buffer!", cid, topic); iterator.remove(); continue; } @@ -645,7 +645,7 @@ private void putAckToStore(final PopCheckPointWrapper pointWrapper, byte msgInde ackMsg.setQueueId(point.getQueueId()); ackMsg.setPopTime(point.getPopTime()); ackMsg.setBrokerName(point.getBrokerName()); - msgInner.setTopic(popMessageProcessor.reviveTopic); + msgInner.setTopic(popMessageProcessor.getReviveTopic()); msgInner.setBody(JSON.toJSONString(ackMsg).getBytes(DataConverter.CHARSET_UTF8)); msgInner.setQueueId(pointWrapper.getReviveQueueId()); msgInner.setTags(PopAckConstants.ACK_TAG); @@ -701,7 +701,7 @@ private void putBatchAckToStore(final PopCheckPointWrapper pointWrapper, final L batchAckMsg.setTopic(point.getTopic()); batchAckMsg.setQueueId(point.getQueueId()); batchAckMsg.setPopTime(point.getPopTime()); - msgInner.setTopic(popMessageProcessor.reviveTopic); + msgInner.setTopic(popMessageProcessor.getReviveTopic()); msgInner.setBody(JSON.toJSONString(batchAckMsg).getBytes(DataConverter.CHARSET_UTF8)); msgInner.setQueueId(pointWrapper.getReviveQueueId()); msgInner.setTags(PopAckConstants.BATCH_ACK_TAG); @@ -751,7 +751,7 @@ private boolean cancelCkTimer(final PopCheckPointWrapper pointWrapper) { } PopCheckPoint point = pointWrapper.getCk(); MessageExtBrokerInner msgInner = new MessageExtBrokerInner(); - msgInner.setTopic(popMessageProcessor.reviveTopic); + msgInner.setTopic(popMessageProcessor.getReviveTopic()); msgInner.setBody((pointWrapper.getReviveQueueId() + "-" + pointWrapper.getReviveQueueOffset()).getBytes(StandardCharsets.UTF_8)); msgInner.setQueueId(pointWrapper.getReviveQueueId()); msgInner.setTags(PopAckConstants.CK_TAG); diff --git a/broker/src/main/java/org/apache/rocketmq/broker/processor/PopMessageProcessor.java b/broker/src/main/java/org/apache/rocketmq/broker/processor/PopMessageProcessor.java index 05efc14b7b4..9355af319ee 100644 --- a/broker/src/main/java/org/apache/rocketmq/broker/processor/PopMessageProcessor.java +++ b/broker/src/main/java/org/apache/rocketmq/broker/processor/PopMessageProcessor.java @@ -47,6 +47,7 @@ import org.apache.rocketmq.broker.longpolling.PopRequest; import org.apache.rocketmq.broker.metrics.BrokerMetricsManager; import org.apache.rocketmq.broker.pagecache.ManyMessageTransfer; +import org.apache.rocketmq.broker.pop.PopConsumerContext; import org.apache.rocketmq.common.BrokerConfig; import org.apache.rocketmq.common.KeyBuilder; import org.apache.rocketmq.common.MixAll; @@ -99,13 +100,12 @@ public class PopMessageProcessor implements NettyRequestProcessor { - private static final Logger POP_LOGGER = - LoggerFactory.getLogger(LoggerName.ROCKETMQ_POP_LOGGER_NAME); + private static final Logger POP_LOGGER = LoggerFactory.getLogger(LoggerName.ROCKETMQ_POP_LOGGER_NAME); + private static final String BORN_TIME = "bornTime"; private final BrokerController brokerController; private final Random random = new Random(System.currentTimeMillis()); - String reviveTopic; - private static final String BORN_TIME = "bornTime"; + private final String reviveTopic; private final PopLongPollingService popLongPollingService; private final PopBufferMergeService popBufferMergeService; @@ -114,13 +114,18 @@ public class PopMessageProcessor implements NettyRequestProcessor { public PopMessageProcessor(final BrokerController brokerController) { this.brokerController = brokerController; - this.reviveTopic = PopAckConstants.buildClusterReviveTopic(this.brokerController.getBrokerConfig().getBrokerClusterName()); + this.reviveTopic = PopAckConstants.buildClusterReviveTopic( + this.brokerController.getBrokerConfig().getBrokerClusterName()); this.popLongPollingService = new PopLongPollingService(brokerController, this, false); this.queueLockManager = new QueueLockManager(); this.popBufferMergeService = new PopBufferMergeService(this.brokerController, this); this.ckMessageNumber = new AtomicLong(); } + protected String getReviveTopic() { + return reviveTopic; + } + public PopLongPollingService getPopLongPollingService() { return popLongPollingService; } @@ -213,27 +218,26 @@ public void notifyMessageArriving(final String topic, final int queueId, final S @Override public RemotingCommand processRequest(final ChannelHandlerContext ctx, RemotingCommand request) throws RemotingCommandException { + final long beginTimeMills = this.brokerController.getMessageStore().now(); + + // fill bron time to properties if not exist, why we need this? request.addExtFieldIfNotExist(BORN_TIME, String.valueOf(System.currentTimeMillis())); if (Objects.equals(request.getExtFields().get(BORN_TIME), "0")) { request.addExtField(BORN_TIME, String.valueOf(System.currentTimeMillis())); } - Channel channel = ctx.channel(); + Channel channel = ctx.channel(); RemotingCommand response = RemotingCommand.createResponseCommand(PopMessageResponseHeader.class); + response.setOpaque(request.getOpaque()); + + final PopMessageRequestHeader requestHeader = + request.decodeCommandCustomHeader(PopMessageRequestHeader.class, true); final PopMessageResponseHeader responseHeader = (PopMessageResponseHeader) response.readCustomHeader(); - final PopMessageRequestHeader requestHeader = request.decodeCommandCustomHeader(PopMessageRequestHeader.class, true); - StringBuilder startOffsetInfo = new StringBuilder(64); - StringBuilder msgOffsetInfo = new StringBuilder(64); - StringBuilder orderCountInfo = null; - if (requestHeader.isOrder()) { - orderCountInfo = new StringBuilder(64); - } - brokerController.getConsumerManager().compensateBasicConsumerInfo(requestHeader.getConsumerGroup(), - ConsumeType.CONSUME_POP, MessageModel.CLUSTERING); - - response.setOpaque(request.getOpaque()); + // Pop mode only supports consumption in cluster load balancing mode + brokerController.getConsumerManager().compensateBasicConsumerInfo( + requestHeader.getConsumerGroup(), ConsumeType.CONSUME_POP, MessageModel.CLUSTERING); if (brokerController.getBrokerConfig().isEnablePopLog()) { POP_LOGGER.info("receive PopMessage request command, {}", request); @@ -245,12 +249,14 @@ public RemotingCommand processRequest(final ChannelHandlerContext ctx, RemotingC this.brokerController.getBrokerConfig().getBrokerIP1())); return response; } + if (!PermName.isReadable(this.brokerController.getBrokerConfig().getBrokerPermission())) { response.setCode(ResponseCode.NO_PERMISSION); response.setRemark(String.format("the broker[%s] pop message is forbidden", this.brokerController.getBrokerConfig().getBrokerIP1())); return response; } + if (requestHeader.getMaxMsgNums() > 32) { response.setCode(ResponseCode.INVALID_PARAMETER); response.setRemark(String.format("the broker[%s] pop message's num is greater than 32", @@ -292,6 +298,7 @@ public RemotingCommand processRequest(final ChannelHandlerContext ctx, RemotingC response.setRemark(errorInfo); return response; } + SubscriptionGroupConfig subscriptionGroupConfig = this.brokerController.getSubscriptionGroupManager().findSubscriptionGroupConfig(requestHeader.getConsumerGroup()); if (null == subscriptionGroupConfig) { @@ -312,21 +319,25 @@ public RemotingCommand processRequest(final ChannelHandlerContext ctx, RemotingC ExpressionMessageFilter messageFilter = null; if (requestHeader.getExp() != null && !requestHeader.getExp().isEmpty()) { try { - subscriptionData = FilterAPI.build(requestHeader.getTopic(), requestHeader.getExp(), requestHeader.getExpType()); - brokerController.getConsumerManager().compensateSubscribeData(requestHeader.getConsumerGroup(), - requestHeader.getTopic(), subscriptionData); - - String retryTopic = KeyBuilder.buildPopRetryTopic(requestHeader.getTopic(), requestHeader.getConsumerGroup(), brokerConfig.isEnableRetryTopicV2()); - SubscriptionData retrySubscriptionData = FilterAPI.build(retryTopic, SubscriptionData.SUB_ALL, requestHeader.getExpType()); - brokerController.getConsumerManager().compensateSubscribeData(requestHeader.getConsumerGroup(), - retryTopic, retrySubscriptionData); + // origin topic + subscriptionData = FilterAPI.build( + requestHeader.getTopic(), requestHeader.getExp(), requestHeader.getExpType()); + brokerController.getConsumerManager().compensateSubscribeData( + requestHeader.getConsumerGroup(), requestHeader.getTopic(), subscriptionData); + + // retry topic + String retryTopic = KeyBuilder.buildPopRetryTopic( + requestHeader.getTopic(), requestHeader.getConsumerGroup(), brokerConfig.isEnableRetryTopicV2()); + SubscriptionData retrySubscriptionData = FilterAPI.build( + retryTopic, SubscriptionData.SUB_ALL, requestHeader.getExpType()); + brokerController.getConsumerManager().compensateSubscribeData( + requestHeader.getConsumerGroup(), retryTopic, retrySubscriptionData); ConsumerFilterData consumerFilterData = null; if (!ExpressionType.isTagType(subscriptionData.getExpressionType())) { consumerFilterData = ConsumerFilterManager.build( requestHeader.getTopic(), requestHeader.getConsumerGroup(), requestHeader.getExp(), - requestHeader.getExpType(), System.currentTimeMillis() - ); + requestHeader.getExpType(), System.currentTimeMillis()); if (consumerFilterData == null) { POP_LOGGER.warn("Parse the consumer's subscription[{}] failed, group: {}", requestHeader.getExp(), requestHeader.getConsumerGroup()); @@ -335,8 +346,8 @@ public RemotingCommand processRequest(final ChannelHandlerContext ctx, RemotingC return response; } } - messageFilter = new ExpressionMessageFilter(subscriptionData, consumerFilterData, - brokerController.getConsumerFilterManager()); + messageFilter = new ExpressionMessageFilter( + subscriptionData, consumerFilterData, brokerController.getConsumerFilterManager()); } catch (Exception e) { POP_LOGGER.warn("Parse the consumer's subscription[{}] error, group: {}", requestHeader.getExp(), requestHeader.getConsumerGroup()); @@ -346,30 +357,139 @@ public RemotingCommand processRequest(final ChannelHandlerContext ctx, RemotingC } } else { try { + // origin topic subscriptionData = FilterAPI.build(requestHeader.getTopic(), "*", ExpressionType.TAG); - brokerController.getConsumerManager().compensateSubscribeData(requestHeader.getConsumerGroup(), - requestHeader.getTopic(), subscriptionData); + brokerController.getConsumerManager().compensateSubscribeData( + requestHeader.getConsumerGroup(), requestHeader.getTopic(), subscriptionData); - String retryTopic = KeyBuilder.buildPopRetryTopic(requestHeader.getTopic(), requestHeader.getConsumerGroup(), brokerConfig.isEnableRetryTopicV2()); + // retry topic + String retryTopic = KeyBuilder.buildPopRetryTopic( + requestHeader.getTopic(), requestHeader.getConsumerGroup(), brokerConfig.isEnableRetryTopicV2()); SubscriptionData retrySubscriptionData = FilterAPI.build(retryTopic, "*", ExpressionType.TAG); - brokerController.getConsumerManager().compensateSubscribeData(requestHeader.getConsumerGroup(), - retryTopic, retrySubscriptionData); + brokerController.getConsumerManager().compensateSubscribeData( + requestHeader.getConsumerGroup(), retryTopic, retrySubscriptionData); } catch (Exception e) { POP_LOGGER.warn("Build default subscription error, group: {}", requestHeader.getConsumerGroup()); } } + GetMessageResult getMessageResult = new GetMessageResult(requestHeader.getMaxMsgNums()); + ExpressionMessageFilter finalMessageFilter = messageFilter; + SubscriptionData finalSubscriptionData = subscriptionData; + + if (brokerConfig.isPopConsumerKVServiceEnable()) { + + CompletableFuture popAsyncFuture = brokerController.getPopConsumerService().popAsync( + RemotingHelper.parseChannelRemoteAddr(channel), beginTimeMills, requestHeader.getInvisibleTime(), + requestHeader.getConsumerGroup(), requestHeader.getTopic(), requestHeader.getQueueId(), + requestHeader.getMaxMsgNums(), requestHeader.isOrder(), requestHeader.getAttemptId(), messageFilter); + + popAsyncFuture.thenApply(result -> { + if (result.isFound()) { + response.setCode(ResponseCode.SUCCESS); + getMessageResult.setStatus(GetMessageStatus.FOUND); + // recursive processing + if (result.getRestCount() > 0) { + popLongPollingService.notifyMessageArriving( + requestHeader.getTopic(), requestHeader.getQueueId(), requestHeader.getConsumerGroup(), + null, 0L, null, null); + } + } else { + POP_LOGGER.debug("Processor not found, polling request, popTime={}, restCount={}", + result.getPopTime(), result.getRestCount()); + + PollingResult pollingResult = popLongPollingService.polling( + ctx, request, new PollingHeader(requestHeader), finalSubscriptionData, finalMessageFilter); + + if (PollingResult.POLLING_SUC == pollingResult) { + // recursive processing + if (result.getRestCount() > 0) { + popLongPollingService.notifyMessageArriving( + requestHeader.getTopic(), requestHeader.getQueueId(), requestHeader.getConsumerGroup(), + null, 0L, null, null); + } + return null; + } else if (PollingResult.POLLING_FULL == pollingResult) { + response.setCode(ResponseCode.POLLING_FULL); + } else { + response.setCode(ResponseCode.POLLING_TIMEOUT); + } + getMessageResult.setStatus(GetMessageStatus.NO_MESSAGE_IN_QUEUE); + } + + responseHeader.setPopTime(result.getPopTime()); + responseHeader.setInvisibleTime(result.getInvisibleTime()); + responseHeader.setReviveQid( + requestHeader.isOrder() ? KeyBuilder.POP_ORDER_REVIVE_QUEUE : 0); + responseHeader.setRestNum(result.getRestCount()); + responseHeader.setStartOffsetInfo(result.getStartOffsetInfo()); + responseHeader.setMsgOffsetInfo(result.getMsgOffsetInfo()); + if (requestHeader.isOrder() && !result.getOrderCountInfo().isEmpty()) { + responseHeader.setOrderCountInfo(result.getOrderCountInfo()); + } + + response.setRemark(getMessageResult.getStatus().name()); + if (response.getCode() != ResponseCode.SUCCESS) { + return response; + } + + // add message + result.getGetMessageResultList().forEach(temp -> { + for (int i = 0; i < temp.getMessageMapedList().size(); i++) { + getMessageResult.addMessage(temp.getMessageMapedList().get(i)); + } + }); + + if (this.brokerController.getBrokerConfig().isTransferMsgByHeap()) { + final byte[] r = this.readGetMessageResult(getMessageResult, + requestHeader.getConsumerGroup(), requestHeader.getTopic(), requestHeader.getQueueId()); + this.brokerController.getBrokerStatsManager().incGroupGetLatency( + requestHeader.getConsumerGroup(), requestHeader.getTopic(), requestHeader.getQueueId(), + (int) (this.brokerController.getMessageStore().now() - beginTimeMills)); + response.setBody(r); + } else { + final GetMessageResult tmpGetMessageResult = getMessageResult; + try { + FileRegion fileRegion = new ManyMessageTransfer( + response.encodeHeader(getMessageResult.getBufferTotalSize()), getMessageResult); + channel.writeAndFlush(fileRegion) + .addListener((ChannelFutureListener) future -> { + tmpGetMessageResult.release(); + Attributes attributes = RemotingMetricsManager.newAttributesBuilder() + .put(LABEL_REQUEST_CODE, RemotingHelper.getRequestCodeDesc(request.getCode())) + .put(LABEL_RESPONSE_CODE, RemotingHelper.getResponseCodeDesc(response.getCode())) + .put(LABEL_RESULT, RemotingMetricsManager.getWriteAndFlushResult(future)) + .build(); + RemotingMetricsManager.rpcLatency.record( + request.getProcessTimer().elapsed(TimeUnit.MILLISECONDS), attributes); + if (!future.isSuccess()) { + POP_LOGGER.error("Fail to transfer messages from page cache to {}", + channel.remoteAddress(), future.cause()); + } + }); + } catch (Throwable e) { + POP_LOGGER.error("Error occurred when transferring messages from page cache", e); + getMessageResult.release(); + } + return null; + } + return response; + }).thenAccept(result -> NettyRemotingAbstract.writeResponse(channel, request, result)); + return null; + } + int randomQ = random.nextInt(100); int reviveQid; if (requestHeader.isOrder()) { reviveQid = KeyBuilder.POP_ORDER_REVIVE_QUEUE; } else { - reviveQid = (int) Math.abs(ckMessageNumber.getAndIncrement() % this.brokerController.getBrokerConfig().getReviveQueueNum()); + reviveQid = (int) Math.abs(ckMessageNumber.getAndIncrement() % + this.brokerController.getBrokerConfig().getReviveQueueNum()); } - GetMessageResult getMessageResult = new GetMessageResult(requestHeader.getMaxMsgNums()); - ExpressionMessageFilter finalMessageFilter = messageFilter; - StringBuilder finalOrderCountInfo = orderCountInfo; + StringBuilder startOffsetInfo = new StringBuilder(64); + StringBuilder msgOffsetInfo = new StringBuilder(64); + StringBuilder orderCountInfo = requestHeader.isOrder() ? new StringBuilder(64) : null; // Due to the design of the fields startOffsetInfo, msgOffsetInfo, and orderCountInfo, // a single POP request could only invoke the popMsgFromQueue method once @@ -404,7 +524,7 @@ public RemotingCommand processRequest(final ChannelHandlerContext ctx, RemotingC getMessageFuture = getMessageFuture.thenCompose(restNum -> popMsgFromQueue(topicConfig.getTopicName(), requestHeader.getAttemptId(), false, getMessageResult, requestHeader, queueId, restNum, reviveQid, channel, popTime, finalMessageFilter, - startOffsetInfo, msgOffsetInfo, finalOrderCountInfo)); + startOffsetInfo, msgOffsetInfo, orderCountInfo)); } // if not full , fetch retry again if (!needRetry && getMessageResult.getMessageMapedList().size() < requestHeader.getMaxMsgNums() && !requestHeader.isOrder()) { @@ -420,7 +540,6 @@ public RemotingCommand processRequest(final ChannelHandlerContext ctx, RemotingC } final RemotingCommand finalResponse = response; - SubscriptionData finalSubscriptionData = subscriptionData; getMessageFuture.thenApply(restNum -> { try { if (request.getCallbackList() != null) { @@ -463,8 +582,8 @@ public RemotingCommand processRequest(final ChannelHandlerContext ctx, RemotingC responseHeader.setRestNum(restNum); responseHeader.setStartOffsetInfo(startOffsetInfo.toString()); responseHeader.setMsgOffsetInfo(msgOffsetInfo.toString()); - if (requestHeader.isOrder() && finalOrderCountInfo != null) { - responseHeader.setOrderCountInfo(finalOrderCountInfo.toString()); + if (requestHeader.isOrder() && orderCountInfo != null) { + responseHeader.setOrderCountInfo(orderCountInfo.toString()); } finalResponse.setRemark(getMessageResult.getStatus().name()); switch (finalResponse.getCode()) { @@ -537,10 +656,12 @@ private CompletableFuture popMsgFromTopic(String topic, boolean isRetry, G messageFilter, startOffsetInfo, msgOffsetInfo, orderCountInfo, randomQ, getMessageFuture); } - private CompletableFuture popMsgFromQueue(String topic, String attemptId, boolean isRetry, GetMessageResult getMessageResult, + private CompletableFuture popMsgFromQueue(String topic, String attemptId, boolean isRetry, + GetMessageResult getMessageResult, PopMessageRequestHeader requestHeader, int queueId, long restNum, int reviveQid, Channel channel, long popTime, ExpressionMessageFilter messageFilter, StringBuilder startOffsetInfo, StringBuilder msgOffsetInfo, StringBuilder orderCountInfo) { + String lockKey = topic + PopAckConstants.SPLIT + requestHeader.getConsumerGroup() + PopAckConstants.SPLIT + queueId; boolean isOrder = requestHeader.isOrder(); @@ -792,7 +913,7 @@ private long getInitOffset(String topic, String group, int queueId, int initMode return offset; } - public final MessageExtBrokerInner buildCkMsg(final PopCheckPoint ck, final int reviveQid) { + public MessageExtBrokerInner buildCkMsg(final PopCheckPoint ck, final int reviveQid) { MessageExtBrokerInner msgInner = new MessageExtBrokerInner(); msgInner.setTopic(reviveTopic); diff --git a/broker/src/main/java/org/apache/rocketmq/broker/processor/RecallMessageProcessor.java b/broker/src/main/java/org/apache/rocketmq/broker/processor/RecallMessageProcessor.java index 7a652f43151..372db0d36eb 100644 --- a/broker/src/main/java/org/apache/rocketmq/broker/processor/RecallMessageProcessor.java +++ b/broker/src/main/java/org/apache/rocketmq/broker/processor/RecallMessageProcessor.java @@ -57,6 +57,12 @@ public RemotingCommand processRequest(ChannelHandlerContext ctx, RemotingCommand final RecallMessageRequestHeader requestHeader = request.decodeCommandCustomHeader(RecallMessageRequestHeader.class); + if (!brokerController.getBrokerConfig().isRecallMessageEnable()) { + response.setCode(ResponseCode.NO_PERMISSION); + response.setRemark("recall failed, operation is forbidden"); + return response; + } + if (BrokerRole.SLAVE == brokerController.getMessageStoreConfig().getBrokerRole()) { response.setCode(ResponseCode.SLAVE_NOT_AVAILABLE); response.setRemark("recall failed, broker service not available"); diff --git a/broker/src/main/java/org/apache/rocketmq/broker/slave/SlaveSynchronize.java b/broker/src/main/java/org/apache/rocketmq/broker/slave/SlaveSynchronize.java index aa77b773ee9..bfb5c9dcd03 100644 --- a/broker/src/main/java/org/apache/rocketmq/broker/slave/SlaveSynchronize.java +++ b/broker/src/main/java/org/apache/rocketmq/broker/slave/SlaveSynchronize.java @@ -17,6 +17,7 @@ package org.apache.rocketmq.broker.slave; import java.io.IOException; +import java.util.concurrent.ConcurrentHashMap; import java.util.concurrent.ConcurrentMap; import org.apache.commons.lang3.StringUtils; @@ -30,8 +31,10 @@ import org.apache.rocketmq.logging.org.slf4j.LoggerFactory; import org.apache.rocketmq.remoting.protocol.body.ConsumerOffsetSerializeWrapper; import org.apache.rocketmq.remoting.protocol.body.MessageRequestModeSerializeWrapper; +import org.apache.rocketmq.remoting.protocol.body.SetMessageRequestModeRequestBody; import org.apache.rocketmq.remoting.protocol.body.SubscriptionGroupWrapper; import org.apache.rocketmq.remoting.protocol.body.TopicConfigAndMappingSerializeWrapper; +import org.apache.rocketmq.remoting.protocol.subscription.SubscriptionGroupConfig; import org.apache.rocketmq.store.config.StorePathConfigHelper; import org.apache.rocketmq.store.timer.TimerCheckpoint; import org.apache.rocketmq.store.timer.TimerMetrics; @@ -166,9 +169,16 @@ private void syncSubscriptionGroupConfig() { this.brokerController.getSubscriptionGroupManager(); subscriptionGroupManager.getDataVersion().assignNewOne( subscriptionWrapper.getDataVersion()); - subscriptionGroupManager.getSubscriptionGroupTable().clear(); - subscriptionGroupManager.getSubscriptionGroupTable().putAll( - subscriptionWrapper.getSubscriptionGroupTable()); + + ConcurrentMap curSubscriptionGroupTable = + subscriptionGroupManager.getSubscriptionGroupTable(); + ConcurrentMap newSubscriptionGroupTable = + subscriptionWrapper.getSubscriptionGroupTable(); + // delete + curSubscriptionGroupTable.entrySet().removeIf(e -> !newSubscriptionGroupTable.containsKey(e.getKey())); + // update + curSubscriptionGroupTable.putAll(newSubscriptionGroupTable); + // persist subscriptionGroupManager.persist(); LOGGER.info("Update slave Subscription Group from master, {}", masterAddrBak); } @@ -187,10 +197,16 @@ private void syncMessageRequestMode() { MessageRequestModeManager messageRequestModeManager = this.brokerController.getQueryAssignmentProcessor().getMessageRequestModeManager(); - messageRequestModeManager.getMessageRequestModeMap().clear(); - messageRequestModeManager.getMessageRequestModeMap().putAll( - messageRequestModeSerializeWrapper.getMessageRequestModeMap() - ); + ConcurrentHashMap> curMessageRequestModeMap = + messageRequestModeManager.getMessageRequestModeMap(); + ConcurrentHashMap> newMessageRequestModeMap = + messageRequestModeSerializeWrapper.getMessageRequestModeMap(); + + // delete + curMessageRequestModeMap.entrySet().removeIf(e -> !newMessageRequestModeMap.containsKey(e.getKey())); + // update + curMessageRequestModeMap.putAll(newMessageRequestModeMap); + // persist messageRequestModeManager.persist(); LOGGER.info("Update slave Message Request Mode from master, {}", masterAddrBak); } catch (Exception e) { diff --git a/broker/src/main/java/org/apache/rocketmq/broker/transaction/queue/TransactionalMessageServiceImpl.java b/broker/src/main/java/org/apache/rocketmq/broker/transaction/queue/TransactionalMessageServiceImpl.java index 9fdfd0a7101..017803c624c 100644 --- a/broker/src/main/java/org/apache/rocketmq/broker/transaction/queue/TransactionalMessageServiceImpl.java +++ b/broker/src/main/java/org/apache/rocketmq/broker/transaction/queue/TransactionalMessageServiceImpl.java @@ -203,9 +203,9 @@ public void check(long transactionTimeout, int transactionCheckMax, log.info("Queue={} process time reach max={}", messageQueue, MAX_PROCESS_TIME_LIMIT); break; } - if (removeMap.containsKey(i)) { + Long removedOpOffset; + if ((removedOpOffset = removeMap.remove(i)) != null) { log.debug("Half offset {} has been committed/rolled back", i); - Long removedOpOffset = removeMap.remove(i); opMsgMap.get(removedOpOffset).remove(i); if (opMsgMap.get(removedOpOffset).size() == 0) { opMsgMap.remove(removedOpOffset); @@ -456,8 +456,8 @@ private boolean checkPrepareQueueOffset(HashMap removeMap, List createCgColdThresholdMapRuntime() { + Map result = new ConcurrentHashMap<>(); + AccAndTimeStamp accAndTimeStamp = new AccAndTimeStamp(new AtomicLong(1L)); + accAndTimeStamp.setCreateTimeMills(1L); + accAndTimeStamp.setLastColdReadTimeMills(1L); + result.put("consumerGroup1", accAndTimeStamp); + return result; + } + + private ConcurrentHashMap createCgColdThresholdMapConfig() { + ConcurrentHashMap result = new ConcurrentHashMap<>(); + result.put("consumerGroup2", 2048L); + return result; + } +} diff --git a/broker/src/test/java/org/apache/rocketmq/broker/longpolling/PopLongPollingServiceTest.java b/broker/src/test/java/org/apache/rocketmq/broker/longpolling/PopLongPollingServiceTest.java index 1f064ec05d1..003bf09842a 100644 --- a/broker/src/test/java/org/apache/rocketmq/broker/longpolling/PopLongPollingServiceTest.java +++ b/broker/src/test/java/org/apache/rocketmq/broker/longpolling/PopLongPollingServiceTest.java @@ -55,20 +55,20 @@ public class PopLongPollingServiceTest { @Mock private BrokerController brokerController; - + @Mock private NettyRequestProcessor processor; - + @Mock private ChannelHandlerContext ctx; - + @Mock private ExecutorService pullMessageExecutor; - + private PopLongPollingService popLongPollingService; - + private final String defaultTopic = "defaultTopic"; - + @Before public void init() { BrokerConfig brokerConfig = new BrokerConfig(); @@ -76,7 +76,7 @@ public void init() { when(brokerController.getBrokerConfig()).thenReturn(brokerConfig); popLongPollingService = spy(new PopLongPollingService(brokerController, processor, true)); } - + @Test public void testNotifyMessageArrivingWithRetryTopic() { int queueId = 0; @@ -84,31 +84,32 @@ public void testNotifyMessageArrivingWithRetryTopic() { popLongPollingService.notifyMessageArrivingWithRetryTopic(defaultTopic, queueId); verify(popLongPollingService, times(1)).notifyMessageArrivingWithRetryTopic(defaultTopic, queueId, -1L, null, 0L, null, null); } - + @Test public void testNotifyMessageArriving() { int queueId = 0; Long tagsCode = 123L; long offset = 123L; long msgStoreTime = System.currentTimeMillis(); - byte[] filterBitMap = new byte[]{0x01}; + byte[] filterBitMap = new byte[] {0x01}; Map properties = new ConcurrentHashMap<>(); doNothing().when(popLongPollingService).notifyMessageArriving(defaultTopic, queueId, offset, tagsCode, msgStoreTime, filterBitMap, properties); popLongPollingService.notifyMessageArrivingWithRetryTopic(defaultTopic, queueId, offset, tagsCode, msgStoreTime, filterBitMap, properties); verify(popLongPollingService).notifyMessageArriving(defaultTopic, queueId, offset, tagsCode, msgStoreTime, filterBitMap, properties); } - + @Test public void testNotifyMessageArrivingValidRequest() throws Exception { String cid = "CID_1"; int queueId = 0; - ConcurrentHashMap> topicCidMap = new ConcurrentHashMap<>(); + ConcurrentLinkedHashMap> topicCidMap = new ConcurrentLinkedHashMap.Builder>() + .maximumWeightedCapacity(10).build(); ConcurrentHashMap cids = new ConcurrentHashMap<>(); cids.put(cid, (byte) 1); topicCidMap.put(defaultTopic, cids); popLongPollingService = new PopLongPollingService(brokerController, processor, true); ConcurrentLinkedHashMap> pollingMap = - new ConcurrentLinkedHashMap.Builder>().maximumWeightedCapacity(this.brokerController.getBrokerConfig().getPopPollingMapSize()).build(); + new ConcurrentLinkedHashMap.Builder>().maximumWeightedCapacity(this.brokerController.getBrokerConfig().getPopPollingMapSize()).build(); Channel channel = mock(Channel.class); when(channel.isActive()).thenReturn(true); PopRequest popRequest = mock(PopRequest.class); @@ -126,19 +127,19 @@ public void testNotifyMessageArrivingValidRequest() throws Exception { boolean actual = popLongPollingService.notifyMessageArriving(defaultTopic, queueId, cid, null, 0, null, null); assertFalse(actual); } - + @Test public void testWakeUpNullRequest() { assertFalse(popLongPollingService.wakeUp(null)); } - + @Test public void testWakeUpIncompleteRequest() { PopRequest request = mock(PopRequest.class); when(request.complete()).thenReturn(false); assertFalse(popLongPollingService.wakeUp(request)); } - + @Test public void testWakeUpInactiveChannel() { PopRequest request = mock(PopRequest.class); @@ -150,7 +151,7 @@ public void testWakeUpInactiveChannel() { when(brokerController.getPullMessageExecutor()).thenReturn(pullMessageExecutor); assertTrue(popLongPollingService.wakeUp(request)); } - + @Test public void testWakeUpValidRequestWithException() throws Exception { PopRequest request = mock(PopRequest.class); @@ -168,7 +169,7 @@ public void testWakeUpValidRequestWithException() throws Exception { captor.getValue().run(); verify(processor).processRequest(any(), any()); } - + @Test public void testPollingNotPolling() { ChannelHandlerContext ctx = mock(ChannelHandlerContext.class); @@ -180,7 +181,7 @@ public void testPollingNotPolling() { PollingResult result = popLongPollingService.polling(ctx, remotingCommand, requestHeader, subscriptionData, messageFilter); assertEquals(PollingResult.NOT_POLLING, result); } - + @Test public void testPollingServicePollingTimeout() throws IllegalAccessException { String cid = "CID_1"; @@ -194,7 +195,8 @@ public void testPollingServicePollingTimeout() throws IllegalAccessException { when(requestHeader.getPollTime()).thenReturn(1000L); when(requestHeader.getTopic()).thenReturn(defaultTopic); when(requestHeader.getConsumerGroup()).thenReturn("defaultGroup"); - ConcurrentHashMap> topicCidMap = new ConcurrentHashMap<>(); + ConcurrentLinkedHashMap> topicCidMap = new ConcurrentLinkedHashMap.Builder>() + .maximumWeightedCapacity(10).build(); ConcurrentHashMap cids = new ConcurrentHashMap<>(); cids.put(cid, (byte) 1); topicCidMap.put(defaultTopic, cids); @@ -202,7 +204,7 @@ public void testPollingServicePollingTimeout() throws IllegalAccessException { PollingResult result = popLongPollingService.polling(ctx, remotingCommand, requestHeader, subscriptionData, messageFilter); assertEquals(PollingResult.POLLING_TIMEOUT, result); } - + @Test public void testPollingPollingSuc() { ChannelHandlerContext ctx = mock(ChannelHandlerContext.class); diff --git a/broker/src/test/java/org/apache/rocketmq/broker/offset/ConsumerOrderInfoManagerTest.java b/broker/src/test/java/org/apache/rocketmq/broker/offset/ConsumerOrderInfoManagerTest.java index 25b418c9344..4414eda54e9 100644 --- a/broker/src/test/java/org/apache/rocketmq/broker/offset/ConsumerOrderInfoManagerTest.java +++ b/broker/src/test/java/org/apache/rocketmq/broker/offset/ConsumerOrderInfoManagerTest.java @@ -21,7 +21,6 @@ import java.util.Map; import java.util.UUID; import java.util.concurrent.ConcurrentHashMap; -import java.util.concurrent.ConcurrentMap; import java.util.concurrent.TimeUnit; import org.apache.rocketmq.broker.BrokerController; import org.apache.rocketmq.broker.subscription.SubscriptionGroupManager; @@ -29,7 +28,6 @@ import org.apache.rocketmq.common.BrokerConfig; import org.apache.rocketmq.common.TopicConfig; import org.apache.rocketmq.remoting.protocol.header.ExtraInfoUtil; -import org.apache.rocketmq.remoting.protocol.subscription.SubscriptionGroupConfig; import org.assertj.core.util.Lists; import org.junit.Before; import org.junit.Test; @@ -384,9 +382,7 @@ public void testAutoCleanAndEncode() { SubscriptionGroupManager subscriptionGroupManager = mock(SubscriptionGroupManager.class); when(brokerController.getSubscriptionGroupManager()).thenReturn(subscriptionGroupManager); - ConcurrentMap subscriptionGroupConfigConcurrentMap = new ConcurrentHashMap<>(); - subscriptionGroupConfigConcurrentMap.put(GROUP, new SubscriptionGroupConfig()); - when(subscriptionGroupManager.getSubscriptionGroupTable()).thenReturn(subscriptionGroupConfigConcurrentMap); + when(subscriptionGroupManager.containsSubscriptionGroup(GROUP)).thenReturn(true); TopicConfig topicConfig = new TopicConfig(TOPIC); when(topicConfigManager.selectTopicConfig(eq(TOPIC))).thenReturn(topicConfig); diff --git a/broker/src/test/java/org/apache/rocketmq/broker/pop/PopConsumerCacheTest.java b/broker/src/test/java/org/apache/rocketmq/broker/pop/PopConsumerCacheTest.java new file mode 100644 index 00000000000..3f6e893a527 --- /dev/null +++ b/broker/src/test/java/org/apache/rocketmq/broker/pop/PopConsumerCacheTest.java @@ -0,0 +1,144 @@ +/* + * 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.rocketmq.broker.pop; + +import java.util.Collections; +import java.util.Queue; +import java.util.concurrent.LinkedBlockingQueue; +import java.util.concurrent.atomic.AtomicInteger; +import org.apache.commons.lang3.reflect.FieldUtils; +import org.apache.rocketmq.broker.BrokerController; +import org.apache.rocketmq.broker.offset.ConsumerOffsetManager; +import org.apache.rocketmq.common.BrokerConfig; +import org.awaitility.Awaitility; +import org.junit.Assert; +import org.junit.Test; +import org.mockito.Mockito; + +import static org.mockito.ArgumentMatchers.any; + +public class PopConsumerCacheTest { + + private final String attemptId = "attemptId"; + private final String topicId = "TopicTest"; + private final String groupId = "GroupTest"; + private final int queueId = 2; + + @Test + public void consumerRecordsTest() { + BrokerConfig brokerConfig = new BrokerConfig(); + brokerConfig.setPopConsumerKVServiceLog(true); + PopConsumerCache.ConsumerRecords consumerRecords = + new PopConsumerCache.ConsumerRecords(brokerConfig, groupId, topicId, queueId); + Assert.assertNotNull(consumerRecords.toString()); + + for (int i = 0; i < 5; i++) { + consumerRecords.write(new PopConsumerRecord(i, groupId, topicId, queueId, 0, + 20000, 100 + i, attemptId)); + } + Assert.assertEquals(100, consumerRecords.getMinOffsetInBuffer()); + Assert.assertEquals(5, consumerRecords.getInFlightRecordCount()); + + for (int i = 0; i < 2; i++) { + consumerRecords.delete(new PopConsumerRecord(i, groupId, topicId, queueId, 0, + 20000, 100 + i, attemptId)); + } + Assert.assertEquals(102, consumerRecords.getMinOffsetInBuffer()); + Assert.assertEquals(3, consumerRecords.getInFlightRecordCount()); + + long bufferTimeout = brokerConfig.getPopCkStayBufferTime(); + Assert.assertEquals(1, consumerRecords.removeExpiredRecords(bufferTimeout + 2).size()); + Assert.assertNull(consumerRecords.removeExpiredRecords(bufferTimeout + 2)); + Assert.assertEquals(2, consumerRecords.removeExpiredRecords(bufferTimeout + 4).size()); + Assert.assertNull(consumerRecords.removeExpiredRecords(bufferTimeout + 4)); + } + + @Test + public void consumerOffsetTest() throws IllegalAccessException { + BrokerController brokerController = Mockito.mock(BrokerController.class); + PopConsumerKVStore consumerKVStore = Mockito.mock(PopConsumerRocksdbStore.class); + PopConsumerLockService consumerLockService = Mockito.mock(PopConsumerLockService.class); + ConsumerOffsetManager consumerOffsetManager = Mockito.mock(ConsumerOffsetManager.class); + Mockito.when(brokerController.getBrokerConfig()).thenReturn(new BrokerConfig()); + Mockito.when(brokerController.getConsumerOffsetManager()).thenReturn(consumerOffsetManager); + Mockito.when(consumerLockService.tryLock(groupId, topicId)).thenReturn(true); + + PopConsumerCache consumerCache = + new PopConsumerCache(brokerController, consumerKVStore, consumerLockService, null); + consumerCache.commitOffset("CommitOffsetTest", groupId, topicId, queueId, 100L); + consumerCache.removeRecords(groupId, topicId, queueId); + + AtomicInteger estimateCacheSize = (AtomicInteger) FieldUtils.readField( + consumerCache, "estimateCacheSize", true); + estimateCacheSize.set(2); + consumerCache.start(); + Awaitility.await().until(() -> estimateCacheSize.get() == 0); + consumerCache.shutdown(); + } + + @Test + public void consumerCacheTest() { + BrokerController brokerController = Mockito.mock(BrokerController.class); + PopConsumerKVStore consumerKVStore = Mockito.mock(PopConsumerRocksdbStore.class); + PopConsumerLockService consumerLockService = Mockito.mock(PopConsumerLockService.class); + Mockito.when(brokerController.getBrokerConfig()).thenReturn(new BrokerConfig()); + + PopConsumerCache consumerCache = + new PopConsumerCache(brokerController, consumerKVStore, consumerLockService, null); + Assert.assertEquals(-1L, consumerCache.getMinOffsetInCache(groupId, topicId, queueId)); + Assert.assertEquals(0, consumerCache.getPopInFlightMessageCount(groupId, topicId, queueId)); + Assert.assertEquals(0, consumerCache.getCacheKeySize()); + + // write + for (int i = 0; i < 3; i++) { + PopConsumerRecord record = new PopConsumerRecord(2L, groupId, topicId, queueId, + 0, 20000, 100 + i, attemptId); + Assert.assertEquals(consumerCache.getKey(record), consumerCache.getKey(groupId, topicId, queueId)); + consumerCache.writeRecords(Collections.singletonList(record)); + } + Assert.assertEquals(100, consumerCache.getMinOffsetInCache(groupId, topicId, queueId)); + Assert.assertEquals(3, consumerCache.getPopInFlightMessageCount(groupId, topicId, queueId)); + Assert.assertEquals(1, consumerCache.getCacheKeySize()); + Assert.assertEquals(3, consumerCache.getCacheSize()); + Assert.assertFalse(consumerCache.isCacheFull()); + + // delete + PopConsumerRecord record = new PopConsumerRecord(2L, groupId, topicId, queueId, + 0, 20000, 100, attemptId); + Assert.assertEquals(0, consumerCache.deleteRecords(Collections.singletonList(record)).size()); + Assert.assertEquals(101, consumerCache.getMinOffsetInCache(groupId, topicId, queueId)); + Assert.assertEquals(2, consumerCache.getPopInFlightMessageCount(groupId, topicId, queueId)); + Assert.assertEquals(2, consumerCache.getCacheSize()); + + record = new PopConsumerRecord(2L, groupId, topicId, queueId, + 0, 20000, 104, attemptId); + Assert.assertEquals(1, consumerCache.deleteRecords(Collections.singletonList(record)).size()); + Assert.assertEquals(101, consumerCache.getMinOffsetInCache(groupId, topicId, queueId)); + Assert.assertEquals(2, consumerCache.getPopInFlightMessageCount(groupId, topicId, queueId)); + + // clean expired records + Queue consumerRecordList = new LinkedBlockingQueue<>(); + consumerCache.cleanupRecords(consumerRecordList::add); + Assert.assertEquals(2, consumerRecordList.size()); + + // clean all + Mockito.when(consumerLockService.isLockTimeout(any(), any())).thenReturn(true); + consumerRecordList.clear(); + consumerCache.cleanupRecords(consumerRecordList::add); + Assert.assertEquals(0, consumerRecordList.size()); + } +} \ No newline at end of file diff --git a/broker/src/test/java/org/apache/rocketmq/broker/pop/PopConsumerContextTest.java b/broker/src/test/java/org/apache/rocketmq/broker/pop/PopConsumerContextTest.java new file mode 100644 index 00000000000..554933eabc4 --- /dev/null +++ b/broker/src/test/java/org/apache/rocketmq/broker/pop/PopConsumerContextTest.java @@ -0,0 +1,69 @@ +/* + * 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.rocketmq.broker.pop; + +import org.apache.rocketmq.store.GetMessageResult; +import org.apache.rocketmq.store.GetMessageStatus; +import org.apache.rocketmq.store.SelectMappedBufferResult; +import org.junit.Assert; +import org.junit.Test; +import org.mockito.Mockito; + +public class PopConsumerContextTest { + + @Test + public void consumerContextTest() { + long popTime = System.currentTimeMillis(); + PopConsumerContext context = new PopConsumerContext("127.0.0.1:6789", + popTime, 20_000, "GroupId", true, "attemptId"); + + Assert.assertFalse(context.isFound()); + Assert.assertEquals("127.0.0.1:6789", context.getClientHost()); + Assert.assertEquals(popTime, context.getPopTime()); + Assert.assertEquals(20_000, context.getInvisibleTime()); + Assert.assertEquals("GroupId", context.getGroupId()); + Assert.assertTrue(context.isFifo()); + Assert.assertEquals("attemptId", context.getAttemptId()); + Assert.assertEquals(0, context.getRestCount()); + + GetMessageResult getMessageResult = new GetMessageResult(); + getMessageResult.setStatus(GetMessageStatus.FOUND); + getMessageResult.setMinOffset(10L); + getMessageResult.setMaxOffset(20L); + getMessageResult.setNextBeginOffset(15L); + getMessageResult.addMessage(Mockito.mock(SelectMappedBufferResult.class), 10); + getMessageResult.addMessage(Mockito.mock(SelectMappedBufferResult.class), 12); + getMessageResult.addMessage(Mockito.mock(SelectMappedBufferResult.class), 13); + + context.addGetMessageResult(getMessageResult, + "TopicId", 3, PopConsumerRecord.RetryType.NORMAL_TOPIC, 1); + + Assert.assertEquals(3, context.getMessageCount()); + Assert.assertEquals( + getMessageResult.getMaxOffset() - getMessageResult.getNextBeginOffset(), context.getRestCount()); + + // check header + Assert.assertNotNull(context.toString()); + Assert.assertEquals("0 3 1", context.getStartOffsetInfo()); + Assert.assertEquals("0 3 10,12,13", context.getMsgOffsetInfo()); + Assert.assertNotNull(context.getOrderCountInfoBuilder()); + Assert.assertEquals("", context.getOrderCountInfo()); + + Assert.assertEquals(1, context.getGetMessageResultList().size()); + Assert.assertEquals(3, context.getPopConsumerRecordList().size()); + } +} diff --git a/broker/src/test/java/org/apache/rocketmq/broker/pop/PopConsumerLockServiceTest.java b/broker/src/test/java/org/apache/rocketmq/broker/pop/PopConsumerLockServiceTest.java new file mode 100644 index 00000000000..b5af2f31798 --- /dev/null +++ b/broker/src/test/java/org/apache/rocketmq/broker/pop/PopConsumerLockServiceTest.java @@ -0,0 +1,60 @@ +/* + * 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.rocketmq.broker.pop; + +import java.lang.reflect.Field; +import java.util.Map; +import java.util.concurrent.TimeUnit; +import org.apache.rocketmq.common.PopAckConstants; +import org.junit.Assert; +import org.junit.Test; + +public class PopConsumerLockServiceTest { + + @Test + @SuppressWarnings("unchecked") + public void consumerLockTest() throws NoSuchFieldException, IllegalAccessException { + String groupId = "groupId"; + String topicId = "topicId"; + + PopConsumerLockService lockService = + new PopConsumerLockService(TimeUnit.MINUTES.toMillis(2)); + + Assert.assertTrue(lockService.tryLock(groupId, topicId)); + Assert.assertFalse(lockService.tryLock(groupId, topicId)); + lockService.unlock(groupId, topicId); + + Assert.assertTrue(lockService.tryLock(groupId, topicId)); + Assert.assertFalse(lockService.tryLock(groupId, topicId)); + Assert.assertFalse(lockService.isLockTimeout(groupId, topicId)); + lockService.removeTimeout(); + + // set expired + Field field = PopConsumerLockService.class.getDeclaredField("lockTable"); + field.setAccessible(true); + Map table = + (Map) field.get(lockService); + + Field lockTime = PopConsumerLockService.TimedLock.class.getDeclaredField("lockTime"); + lockTime.setAccessible(true); + lockTime.set(table.get(groupId + PopAckConstants.SPLIT + topicId), + System.currentTimeMillis() - TimeUnit.MINUTES.toMillis(3)); + lockService.removeTimeout(); + + Assert.assertEquals(0, table.size()); + } +} \ No newline at end of file diff --git a/broker/src/test/java/org/apache/rocketmq/broker/pop/PopConsumerRecordTest.java b/broker/src/test/java/org/apache/rocketmq/broker/pop/PopConsumerRecordTest.java new file mode 100644 index 00000000000..24a79b33f31 --- /dev/null +++ b/broker/src/test/java/org/apache/rocketmq/broker/pop/PopConsumerRecordTest.java @@ -0,0 +1,75 @@ +/* + * 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.rocketmq.broker.pop; + +import java.util.UUID; +import org.apache.rocketmq.common.constant.LoggerName; +import org.apache.rocketmq.logging.org.slf4j.Logger; +import org.apache.rocketmq.logging.org.slf4j.LoggerFactory; +import org.junit.Assert; +import org.junit.Test; + +public class PopConsumerRecordTest { + + private static final Logger log = LoggerFactory.getLogger(LoggerName.ROCKETMQ_POP_LOGGER_NAME); + + @Test + public void retryCodeTest() { + Assert.assertEquals("NORMAL_TOPIC code should be 0", + 0, PopConsumerRecord.RetryType.NORMAL_TOPIC.getCode()); + Assert.assertEquals("RETRY_TOPIC code should be 1", + 1, PopConsumerRecord.RetryType.RETRY_TOPIC_V1.getCode()); + Assert.assertEquals("RETRY_TOPIC_V2 code should be 2", + 2, PopConsumerRecord.RetryType.RETRY_TOPIC_V2.getCode()); + } + + @Test + public void deliveryRecordSerializeTest() { + PopConsumerRecord consumerRecord = new PopConsumerRecord(); + consumerRecord.setPopTime(System.currentTimeMillis()); + consumerRecord.setGroupId("GroupId"); + consumerRecord.setTopicId("TopicId"); + consumerRecord.setQueueId(3); + consumerRecord.setRetryFlag(PopConsumerRecord.RetryType.RETRY_TOPIC_V1.getCode()); + consumerRecord.setInvisibleTime(20); + consumerRecord.setOffset(100); + consumerRecord.setAttemptTimes(2); + consumerRecord.setAttemptId(UUID.randomUUID().toString().toUpperCase()); + + Assert.assertTrue(consumerRecord.isRetry()); + Assert.assertEquals(consumerRecord.getPopTime() + consumerRecord.getInvisibleTime(), + consumerRecord.getVisibilityTimeout()); + Assert.assertEquals(8 + "GroupId".length() + 1 + "TopicId".length() + 1 + 4 + 1 + 8, + consumerRecord.getKeyBytes().length); + log.info("ConsumerRecord={}", consumerRecord.toString()); + + PopConsumerRecord decodeRecord = PopConsumerRecord.decode(consumerRecord.getValueBytes()); + PopConsumerRecord consumerRecord2 = new PopConsumerRecord(consumerRecord.getPopTime(), + consumerRecord.getGroupId(), consumerRecord.getTopicId(), consumerRecord.getQueueId(), + consumerRecord.getRetryFlag(), consumerRecord.getInvisibleTime(), + consumerRecord.getOffset(), consumerRecord.getAttemptId()); + Assert.assertEquals(decodeRecord.getPopTime(), consumerRecord2.getPopTime()); + Assert.assertEquals(decodeRecord.getGroupId(), consumerRecord2.getGroupId()); + Assert.assertEquals(decodeRecord.getTopicId(), consumerRecord2.getTopicId()); + Assert.assertEquals(decodeRecord.getQueueId(), consumerRecord2.getQueueId()); + Assert.assertEquals(decodeRecord.getRetryFlag(), consumerRecord2.getRetryFlag()); + Assert.assertEquals(decodeRecord.getInvisibleTime(), consumerRecord2.getInvisibleTime()); + Assert.assertEquals(decodeRecord.getOffset(), consumerRecord2.getOffset()); + Assert.assertEquals(0, consumerRecord2.getAttemptTimes()); + Assert.assertEquals(decodeRecord.getAttemptId(), consumerRecord2.getAttemptId()); + } +} \ No newline at end of file diff --git a/broker/src/test/java/org/apache/rocketmq/broker/pop/PopConsumerRocksdbStoreTest.java b/broker/src/test/java/org/apache/rocketmq/broker/pop/PopConsumerRocksdbStoreTest.java new file mode 100644 index 00000000000..3c2b190d1cd --- /dev/null +++ b/broker/src/test/java/org/apache/rocketmq/broker/pop/PopConsumerRocksdbStoreTest.java @@ -0,0 +1,193 @@ +/* + * 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.rocketmq.broker.pop; + +import com.google.common.base.Stopwatch; +import java.io.File; +import java.io.IOException; +import java.lang.reflect.Field; +import java.nio.file.Paths; +import java.util.ArrayList; +import java.util.List; +import java.util.UUID; +import java.util.concurrent.TimeUnit; +import java.util.stream.Collectors; +import java.util.stream.IntStream; +import org.apache.commons.io.FileUtils; +import org.apache.rocketmq.common.config.AbstractRocksDBStorage; +import org.apache.rocketmq.common.constant.LoggerName; +import org.apache.rocketmq.tieredstore.util.MessageStoreUtil; +import org.junit.Assert; +import org.junit.Ignore; +import org.junit.Test; +import org.rocksdb.RocksDB; +import org.rocksdb.RocksIterator; +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; + +public class PopConsumerRocksdbStoreTest { + + private static final Logger log = LoggerFactory.getLogger(LoggerName.ROCKETMQ_POP_LOGGER_NAME); + private static final String CONSUMER_STORE_PATH = "consumer_rocksdb"; + + public static String getRandomStorePath() { + return Paths.get(System.getProperty("user.home"), "store_test", CONSUMER_STORE_PATH, + UUID.randomUUID().toString().replace("-", "").toUpperCase().substring(0, 16)).toString(); + } + + public static void deleteStoreDirectory(String storePath) { + try { + FileUtils.deleteDirectory(new File(storePath)); + } catch (IOException e) { + log.error("Delete store directory failed, filePath: {}", storePath, e); + } + } + + public static PopConsumerRecord getConsumerRecord() { + return new PopConsumerRecord(1L, "GroupTest", "TopicTest", 2, + PopConsumerRecord.RetryType.NORMAL_TOPIC.getCode(), TimeUnit.SECONDS.toMillis(20), 100L, "AttemptId"); + } + + @Test + public void rocksdbStoreWriteDeleteTest() { + String filePath = getRandomStorePath(); + PopConsumerKVStore consumerStore = new PopConsumerRocksdbStore(filePath); + Assert.assertEquals(filePath, consumerStore.getFilePath()); + + consumerStore.start(); + consumerStore.writeRecords(IntStream.range(0, 3).boxed() + .flatMap(i -> + IntStream.range(0, 5).mapToObj(j -> { + PopConsumerRecord consumerRecord = getConsumerRecord(); + consumerRecord.setPopTime(j); + consumerRecord.setQueueId(i); + consumerRecord.setOffset(100L + j); + return consumerRecord; + }) + ) + .collect(Collectors.toList())); + consumerStore.deleteRecords(IntStream.range(0, 2).boxed() + .flatMap(i -> + IntStream.range(0, 5).mapToObj(j -> { + PopConsumerRecord consumerRecord = getConsumerRecord(); + consumerRecord.setPopTime(j); + consumerRecord.setQueueId(i); + consumerRecord.setOffset(100L + j); + return consumerRecord; + }) + ) + .collect(Collectors.toList())); + + List consumerRecords = + consumerStore.scanExpiredRecords(0, 20002, 2); + Assert.assertEquals(2, consumerRecords.size()); + consumerStore.deleteRecords(consumerRecords); + + consumerRecords = consumerStore.scanExpiredRecords(0, 20003, 2); + Assert.assertEquals(1, consumerRecords.size()); + consumerStore.deleteRecords(consumerRecords); + + consumerRecords = consumerStore.scanExpiredRecords(0, 20005, 3); + Assert.assertEquals(2, consumerRecords.size()); + + consumerStore.shutdown(); + deleteStoreDirectory(filePath); + } + + private long getDirectorySizeRecursive(File directory) { + long size = 0; + File[] files = directory.listFiles(); + if (files != null) { + for (File file : files) { + if (file.isFile()) { + size += file.length(); + } else if (file.isDirectory()) { + size += getDirectorySizeRecursive(file); + } + } + } + return size; + } + + @Test + @Ignore + @SuppressWarnings("ConstantValue") + public void tombstoneDeletionTest() throws IllegalAccessException, NoSuchFieldException { + PopConsumerRocksdbStore rocksdbStore = new PopConsumerRocksdbStore(getRandomStorePath()); + rocksdbStore.start(); + + int iterCount = 1000 * 1000; + boolean useSeekFirstDelete = false; + Field dbField = AbstractRocksDBStorage.class.getDeclaredField("db"); + dbField.setAccessible(true); + RocksDB rocksDB = (RocksDB) dbField.get(rocksdbStore); + + long currentTime = 0L; + Stopwatch stopwatch = Stopwatch.createStarted(); + for (int i = 0; i < iterCount; i++) { + List records = new ArrayList<>(); + for (int j = 0; j < 1000; j++) { + PopConsumerRecord record = getConsumerRecord(); + record.setPopTime((long) i * iterCount + j); + record.setGroupId("GroupTest"); + record.setTopicId("TopicTest"); + record.setQueueId(i % 10); + record.setRetryFlag(0); + record.setInvisibleTime(TimeUnit.SECONDS.toMillis(30)); + record.setOffset(i); + records.add(record); + } + rocksdbStore.writeRecords(records); + + long start = stopwatch.elapsed(TimeUnit.MILLISECONDS); + List deleteList = new ArrayList<>(); + if (useSeekFirstDelete) { + try (RocksIterator iterator = rocksDB.newIterator(rocksdbStore.columnFamilyHandle)) { + iterator.seekToFirst(); + if (i % 10 == 0) { + long fileSize = getDirectorySizeRecursive(new File(rocksdbStore.getFilePath())); + log.info("DirectorySize={}, Cost={}ms", + MessageStoreUtil.toHumanReadable(fileSize), stopwatch.elapsed(TimeUnit.MILLISECONDS) - start); + } + while (iterator.isValid() && deleteList.size() < 1024) { + deleteList.add(PopConsumerRecord.decode(iterator.value())); + iterator.next(); + } + } + } else { + long upper = System.currentTimeMillis(); + deleteList = rocksdbStore.scanExpiredRecords(currentTime, upper, 800); + if (!deleteList.isEmpty()) { + currentTime = deleteList.get(deleteList.size() - 1).getVisibilityTimeout(); + } + long scanCost = stopwatch.elapsed(TimeUnit.MILLISECONDS) - start; + if (i % 100 == 0) { + long fileSize = getDirectorySizeRecursive(new File(rocksdbStore.getFilePath())); + long seekTime = stopwatch.elapsed(TimeUnit.MILLISECONDS); + try (RocksIterator iterator = rocksDB.newIterator(rocksdbStore.columnFamilyHandle)) { + iterator.seekToFirst(); + } + log.info("DirectorySize={}, Cost={}ms, SeekFirstCost={}ms", MessageStoreUtil.toHumanReadable(fileSize), + scanCost, stopwatch.elapsed(TimeUnit.MILLISECONDS) - seekTime); + } + } + rocksdbStore.deleteRecords(deleteList); + } + rocksdbStore.shutdown(); + deleteStoreDirectory(rocksdbStore.getFilePath()); + } +} \ No newline at end of file diff --git a/broker/src/test/java/org/apache/rocketmq/broker/pop/PopConsumerServiceTest.java b/broker/src/test/java/org/apache/rocketmq/broker/pop/PopConsumerServiceTest.java new file mode 100644 index 00000000000..2b930d5852c --- /dev/null +++ b/broker/src/test/java/org/apache/rocketmq/broker/pop/PopConsumerServiceTest.java @@ -0,0 +1,453 @@ +/* + * 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.rocketmq.broker.pop; + +import java.io.File; +import java.io.IOException; +import java.net.InetSocketAddress; +import java.nio.ByteBuffer; +import java.util.Collections; +import java.util.List; +import java.util.UUID; +import java.util.concurrent.CompletableFuture; +import java.util.concurrent.TimeUnit; +import java.util.concurrent.atomic.AtomicLong; +import java.util.stream.Collectors; +import java.util.stream.IntStream; +import org.apache.commons.io.FileUtils; +import org.apache.commons.lang3.reflect.FieldUtils; +import org.apache.commons.lang3.tuple.Triple; +import org.apache.rocketmq.broker.BrokerController; +import org.apache.rocketmq.broker.failover.EscapeBridge; +import org.apache.rocketmq.broker.longpolling.PopLongPollingService; +import org.apache.rocketmq.broker.offset.ConsumerOffsetManager; +import org.apache.rocketmq.broker.offset.ConsumerOrderInfoManager; +import org.apache.rocketmq.broker.processor.PopMessageProcessor; +import org.apache.rocketmq.broker.topic.TopicConfigManager; +import org.apache.rocketmq.common.BrokerConfig; +import org.apache.rocketmq.common.KeyBuilder; +import org.apache.rocketmq.common.TopicConfig; +import org.apache.rocketmq.common.constant.PermName; +import org.apache.rocketmq.common.message.MessageDecoder; +import org.apache.rocketmq.common.message.MessageExt; +import org.apache.rocketmq.common.message.MessageExtBrokerInner; +import org.apache.rocketmq.store.AppendMessageResult; +import org.apache.rocketmq.store.AppendMessageStatus; +import org.apache.rocketmq.store.GetMessageResult; +import org.apache.rocketmq.store.GetMessageStatus; +import org.apache.rocketmq.store.MessageStore; +import org.apache.rocketmq.store.PutMessageResult; +import org.apache.rocketmq.store.PutMessageStatus; +import org.apache.rocketmq.store.SelectMappedBufferResult; +import org.apache.rocketmq.store.config.MessageStoreConfig; +import org.apache.rocketmq.store.exception.ConsumeQueueException; +import org.apache.rocketmq.store.stats.BrokerStatsManager; +import org.junit.After; +import org.junit.Assert; +import org.junit.Before; +import org.junit.Test; +import org.mockito.Mockito; + +import static org.mockito.ArgumentMatchers.any; +import static org.mockito.ArgumentMatchers.anyInt; +import static org.mockito.ArgumentMatchers.anyLong; +import static org.mockito.ArgumentMatchers.anyString; + +public class PopConsumerServiceTest { + + private final String clientHost = "127.0.0.1:8888"; + private final String groupId = "groupId"; + private final String topicId = "topicId"; + private final int queueId = 2; + private final String attemptId = UUID.randomUUID().toString().toUpperCase(); + private final String filePath = PopConsumerRocksdbStoreTest.getRandomStorePath(); + + private BrokerController brokerController; + private PopConsumerService consumerService; + + @Before + public void init() throws IOException { + BrokerConfig brokerConfig = new BrokerConfig(); + brokerConfig.setEnablePopLog(true); + brokerConfig.setEnablePopBufferMerge(true); + brokerConfig.setEnablePopMessageThreshold(true); + brokerConfig.setPopInflightMessageThreshold(100); + brokerConfig.setPopConsumerKVServiceLog(true); + brokerConfig.setEnableRetryTopicV2(true); + MessageStoreConfig messageStoreConfig = new MessageStoreConfig(); + messageStoreConfig.setStorePathRootDir(filePath); + + TopicConfigManager topicConfigManager = Mockito.mock(TopicConfigManager.class); + ConsumerOffsetManager consumerOffsetManager = Mockito.mock(ConsumerOffsetManager.class); + PopMessageProcessor popMessageProcessor = Mockito.mock(PopMessageProcessor.class); + PopLongPollingService popLongPollingService = Mockito.mock(PopLongPollingService.class); + ConsumerOrderInfoManager consumerOrderInfoManager = Mockito.mock(ConsumerOrderInfoManager.class); + + brokerController = Mockito.mock(BrokerController.class); + Mockito.when(brokerController.getBrokerConfig()).thenReturn(brokerConfig); + Mockito.when(brokerController.getTopicConfigManager()).thenReturn(topicConfigManager); + Mockito.when(brokerController.getMessageStoreConfig()).thenReturn(messageStoreConfig); + Mockito.when(brokerController.getConsumerOffsetManager()).thenReturn(consumerOffsetManager); + Mockito.when(brokerController.getPopMessageProcessor()).thenReturn(popMessageProcessor); + Mockito.when(popMessageProcessor.getPopLongPollingService()).thenReturn(popLongPollingService); + Mockito.when(brokerController.getConsumerOrderInfoManager()).thenReturn(consumerOrderInfoManager); + + consumerService = new PopConsumerService(brokerController); + } + + @After + public void shutdown() throws IOException { + FileUtils.deleteDirectory(new File(filePath)); + } + + public PopConsumerRecord getConsumerTestRecord() { + PopConsumerRecord popConsumerRecord = new PopConsumerRecord(); + popConsumerRecord.setPopTime(System.currentTimeMillis()); + popConsumerRecord.setGroupId(groupId); + popConsumerRecord.setTopicId(topicId); + popConsumerRecord.setQueueId(queueId); + popConsumerRecord.setRetryFlag(PopConsumerRecord.RetryType.NORMAL_TOPIC.getCode()); + popConsumerRecord.setAttemptTimes(0); + popConsumerRecord.setInvisibleTime(TimeUnit.SECONDS.toMillis(20)); + popConsumerRecord.setAttemptId(UUID.randomUUID().toString().toUpperCase()); + return popConsumerRecord; + } + + @Test + public void isPopShouldStopTest() throws IllegalAccessException { + Assert.assertFalse(consumerService.isPopShouldStop(groupId, topicId, queueId)); + PopConsumerCache consumerCache = (PopConsumerCache) FieldUtils.readField( + consumerService, "popConsumerCache", true); + for (int i = 0; i < 100; i++) { + PopConsumerRecord record = getConsumerTestRecord(); + record.setOffset(i); + consumerCache.writeRecords(Collections.singletonList(record)); + } + Assert.assertTrue(consumerService.isPopShouldStop(groupId, topicId, queueId)); + } + + @Test + public void pendingFilterCountTest() throws ConsumeQueueException { + MessageStore messageStore = Mockito.mock(MessageStore.class); + Mockito.when(messageStore.getMaxOffsetInQueue(topicId, queueId)).thenReturn(100L); + Mockito.when(brokerController.getMessageStore()).thenReturn(messageStore); + ConsumerOffsetManager consumerOffsetManager = brokerController.getConsumerOffsetManager(); + Mockito.when(consumerOffsetManager.queryOffset(groupId, topicId, queueId)).thenReturn(20L); + Assert.assertEquals(consumerService.getPendingFilterCount(groupId, topicId, queueId), 80L); + } + + private MessageExt getMessageExt() { + MessageExt messageExt = new MessageExt(); + messageExt.setTopic(topicId); + messageExt.setQueueId(queueId); + messageExt.setBody(new byte[128]); + messageExt.setBornHost(new InetSocketAddress("127.0.0.1", 8080)); + messageExt.setStoreHost(new InetSocketAddress("127.0.0.1", 8080)); + messageExt.putUserProperty("Key", "Value"); + return messageExt; + } + + @Test + public void recodeRetryMessageTest() throws Exception { + GetMessageResult getMessageResult = new GetMessageResult(); + getMessageResult.setStatus(GetMessageStatus.FOUND); + + // result is empty + SelectMappedBufferResult bufferResult = new SelectMappedBufferResult( + 0, ByteBuffer.allocate(10), 10, null); + getMessageResult.addMessage(bufferResult); + getMessageResult.getMessageMapedList().clear(); + GetMessageResult result = consumerService.recodeRetryMessage( + getMessageResult, topicId, 0, 100, 200); + Assert.assertEquals(0, result.getMessageMapedList().size()); + + ByteBuffer buffer = ByteBuffer.wrap( + MessageDecoder.encode(getMessageExt(), false)); + getMessageResult = new GetMessageResult(); + getMessageResult.setStatus(GetMessageStatus.FOUND); + getMessageResult.addMessage(new SelectMappedBufferResult( + 0, buffer, buffer.remaining(), null)); + result = consumerService.recodeRetryMessage( + getMessageResult, topicId, 0, 100, 200); + Assert.assertNotNull(result); + Assert.assertEquals(1, result.getMessageMapedList().size()); + } + + @Test + public void addGetMessageResultTest() { + PopConsumerContext context = new PopConsumerContext( + clientHost, System.currentTimeMillis(), 20000, groupId, false, attemptId); + GetMessageResult result = new GetMessageResult(); + result.setStatus(GetMessageStatus.FOUND); + result.getMessageQueueOffset().add(100L); + consumerService.addGetMessageResult( + context, result, topicId, queueId, PopConsumerRecord.RetryType.NORMAL_TOPIC, 100); + Assert.assertEquals(1, context.getGetMessageResultList().size()); + } + + @Test + public void getMessageAsyncTest() throws Exception { + MessageStore messageStore = Mockito.mock(MessageStore.class); + Mockito.when(brokerController.getMessageStore()).thenReturn(messageStore); + Mockito.when(messageStore.getMessageAsync(groupId, topicId, queueId, 0, 10, null)) + .thenReturn(CompletableFuture.completedFuture(null)); + GetMessageResult getMessageResult = consumerService.getMessageAsync( + "127.0.0.1:8888", groupId, topicId, queueId, 0, 10, null).join(); + Assert.assertNull(getMessageResult); + + // success when first get message + GetMessageResult firstGetMessageResult = new GetMessageResult(); + firstGetMessageResult.setStatus(GetMessageStatus.FOUND); + Mockito.when(messageStore.getMessageAsync(groupId, topicId, queueId, 0, 10, null)) + .thenReturn(CompletableFuture.completedFuture(firstGetMessageResult)); + getMessageResult = consumerService.getMessageAsync( + "127.0.0.1:8888", groupId, topicId, queueId, 0, 10, null).join(); + Assert.assertEquals(GetMessageStatus.FOUND, getMessageResult.getStatus()); + + // reset offset from server + firstGetMessageResult.setStatus(GetMessageStatus.OFFSET_FOUND_NULL); + firstGetMessageResult.setNextBeginOffset(25); + GetMessageResult resetGetMessageResult = new GetMessageResult(); + resetGetMessageResult.setStatus(GetMessageStatus.FOUND); + Mockito.when(messageStore.getMessageAsync(groupId, topicId, queueId, 25, 10, null)) + .thenReturn(CompletableFuture.completedFuture(resetGetMessageResult)); + getMessageResult = consumerService.getMessageAsync( + "127.0.0.1:8888", groupId, topicId, queueId, 0, 10, null).join(); + Assert.assertEquals(GetMessageStatus.FOUND, getMessageResult.getStatus()); + + // fifo block + PopConsumerContext context = new PopConsumerContext( + clientHost, System.currentTimeMillis(), 20000, groupId, false, attemptId); + consumerService.setFifoBlocked(context, groupId, topicId, queueId, Collections.singletonList(100L)); + Mockito.when(brokerController.getConsumerOrderInfoManager() + .checkBlock(anyString(), anyString(), anyString(), anyInt(), anyLong())).thenReturn(true); + Assert.assertTrue(consumerService.isFifoBlocked(context, groupId, topicId, queueId)); + + // get message async normal + CompletableFuture future = CompletableFuture.completedFuture(context); + Assert.assertEquals(0L, consumerService.getMessageAsync(future, clientHost, groupId, topicId, queueId, + 10, null, PopConsumerRecord.RetryType.NORMAL_TOPIC).join().getRestCount()); + + // get message result full, no need get again + for (int i = 0; i < 10; i++) { + ByteBuffer buffer = ByteBuffer.wrap(MessageDecoder.encode(getMessageExt(), false)); + getMessageResult.addMessage(new SelectMappedBufferResult( + 0, buffer, buffer.remaining(), null), i); + } + context.addGetMessageResult(getMessageResult, topicId, queueId, PopConsumerRecord.RetryType.NORMAL_TOPIC, 0); + + Mockito.when(brokerController.getMessageStore().getMaxOffsetInQueue(topicId, queueId)).thenReturn(100L); + Mockito.when(brokerController.getConsumerOffsetManager().queryOffset(groupId, topicId, queueId)).thenReturn(0L); + Assert.assertEquals(100L, consumerService.getMessageAsync(future, clientHost, groupId, topicId, queueId, + 10, null, PopConsumerRecord.RetryType.NORMAL_TOPIC).join().getRestCount()); + + // fifo block test + context = new PopConsumerContext( + clientHost, System.currentTimeMillis(), 20000, groupId, true, attemptId); + future = CompletableFuture.completedFuture(context); + Assert.assertEquals(0L, consumerService.getMessageAsync(future, clientHost, groupId, topicId, queueId, + 10, null, PopConsumerRecord.RetryType.NORMAL_TOPIC).join().getRestCount()); + } + + @Test + public void popAsyncTest() { + PopConsumerService consumerServiceSpy = Mockito.spy(consumerService); + TopicConfigManager topicConfigManager = Mockito.mock(TopicConfigManager.class); + Mockito.when(topicConfigManager.selectTopicConfig(topicId)).thenReturn(new TopicConfig( + topicId, 2, 2, PermName.PERM_READ | PermName.PERM_WRITE, 0)); + Mockito.when(brokerController.getTopicConfigManager()).thenReturn(topicConfigManager); + + String[] retryTopic = new String[] { + KeyBuilder.buildPopRetryTopicV1(topicId, groupId), + KeyBuilder.buildPopRetryTopicV2(topicId, groupId) + }; + + for (String retry : retryTopic) { + GetMessageResult getMessageResult = new GetMessageResult(); + getMessageResult.setStatus(GetMessageStatus.NO_MATCHED_MESSAGE); + getMessageResult.setMinOffset(0L); + getMessageResult.setMaxOffset(1L); + getMessageResult.setNextBeginOffset(1L); + Mockito.doReturn(CompletableFuture.completedFuture(getMessageResult)) + .when(consumerServiceSpy).getMessageAsync(clientHost, groupId, retry, 0, 0, 10, null); + Mockito.doReturn(CompletableFuture.completedFuture(getMessageResult)) + .when(consumerServiceSpy).getMessageAsync(clientHost, groupId, retry, 0, 0, 8, null); + } + + for (int i = -1; i < 2; i++) { + GetMessageResult getMessageResult = new GetMessageResult(); + getMessageResult.setStatus(GetMessageStatus.FOUND); + getMessageResult.setMinOffset(0L); + getMessageResult.setMaxOffset(1L); + getMessageResult.setNextBeginOffset(1L); + getMessageResult.addMessage(Mockito.mock(SelectMappedBufferResult.class), 1L); + + Mockito.doReturn(CompletableFuture.completedFuture(getMessageResult)) + .when(consumerServiceSpy).getMessageAsync(clientHost, groupId, topicId, i, 0, 8, null); + Mockito.doReturn(CompletableFuture.completedFuture(getMessageResult)) + .when(consumerServiceSpy).getMessageAsync(clientHost, groupId, topicId, i, 0, 9, null); + Mockito.doReturn(CompletableFuture.completedFuture(getMessageResult)) + .when(consumerServiceSpy).getMessageAsync(clientHost, groupId, topicId, i, 0, 10, null); + } + + // pop broker + consumerServiceSpy.popAsync(clientHost, System.currentTimeMillis(), + 20000, groupId, topicId, -1, 10, false, attemptId, null).join(); + } + + @Test + public void ackAsyncTest() { + long current = System.currentTimeMillis(); + consumerService.getPopConsumerStore().start(); + consumerService.ackAsync( + current, 10, groupId, topicId, queueId, 100).join(); + consumerService.changeInvisibilityDuration(current, 10, + current + 100, 10, groupId, topicId, queueId, 100); + consumerService.shutdown(); + } + + @Test + public void reviveRetryTest() { + Mockito.when(brokerController.getTopicConfigManager().selectTopicConfig(topicId)).thenReturn(null); + Mockito.when(brokerController.getConsumerOffsetManager().queryOffset(groupId, topicId, 0)).thenReturn(-1L); + + consumerService.createRetryTopicIfNeeded(groupId, topicId); + consumerService.clearCache(groupId, topicId, queueId); + MessageExt messageExt = new MessageExt(); + messageExt.setBody("body".getBytes()); + messageExt.setBornTimestamp(System.currentTimeMillis()); + messageExt.setFlag(0); + messageExt.setSysFlag(0); + messageExt.setReconsumeTimes(1); + messageExt.putUserProperty("key", "value"); + + PopConsumerRecord record = new PopConsumerRecord(); + record.setTopicId("topic"); + record.setGroupId("group"); + Mockito.when(brokerController.getBrokerStatsManager()).thenReturn(Mockito.mock(BrokerStatsManager.class)); + Mockito.when(brokerController.getEscapeBridge()).thenReturn(Mockito.mock(EscapeBridge.class)); + Mockito.when(brokerController.getEscapeBridge().putMessageToSpecificQueue(any(MessageExtBrokerInner.class))) + .thenReturn(new PutMessageResult( + PutMessageStatus.PUT_OK, new AppendMessageResult(AppendMessageStatus.PUT_OK))); + + PopConsumerService consumerServiceSpy = Mockito.spy(consumerService); + Mockito.doNothing().when(consumerServiceSpy).createRetryTopicIfNeeded(any(), any()); + Assert.assertTrue(consumerServiceSpy.reviveRetry(record, messageExt)); + + // write message error + Mockito.when(brokerController.getEscapeBridge().putMessageToSpecificQueue(any(MessageExtBrokerInner.class))) + .thenReturn(new PutMessageResult(PutMessageStatus.UNKNOWN_ERROR, + new AppendMessageResult(AppendMessageStatus.UNKNOWN_ERROR))); + Assert.assertFalse(consumerServiceSpy.reviveRetry(record, messageExt)); + + // revive backoff + consumerService.getPopConsumerStore().start(); + List consumerRecordList = IntStream.range(0, 3) + .mapToObj(i -> { + PopConsumerRecord temp = new PopConsumerRecord(); + temp.setPopTime(0); + temp.setInvisibleTime(20 * 1000); + temp.setTopicId("topic"); + temp.setGroupId("group"); + temp.setQueueId(2); + temp.setOffset(i); + return temp; + }) + .collect(Collectors.toList()); + consumerService.getPopConsumerStore().writeRecords(consumerRecordList); + + Mockito.doReturn(CompletableFuture.completedFuture(null)) + .when(consumerServiceSpy).getMessageAsync(any(PopConsumerRecord.class)); + consumerServiceSpy.revive(new AtomicLong(20 * 1000), 1); + + Mockito.doReturn(CompletableFuture.completedFuture( + Triple.of(null, "GetMessageResult is null", false))) + .when(consumerServiceSpy).getMessageAsync(any(PopConsumerRecord.class)); + consumerServiceSpy.revive(new AtomicLong(20 * 1000), 1); + + Mockito.doReturn(CompletableFuture.completedFuture( + Triple.of(Mockito.mock(MessageExt.class), null, false))) + .when(consumerServiceSpy).getMessageAsync(any(PopConsumerRecord.class)); + consumerServiceSpy.revive(new AtomicLong(20 * 1000), 1); + consumerService.shutdown(); + } + + @Test + public void reviveBackoffRetryTest() { + Mockito.when(brokerController.getEscapeBridge()).thenReturn(Mockito.mock(EscapeBridge.class)); + PopConsumerService consumerServiceSpy = Mockito.spy(consumerService); + + consumerService.getPopConsumerStore().start(); + + long popTime = 1000000000L; + long invisibleTime = 60 * 1000L; + PopConsumerRecord record = new PopConsumerRecord(); + record.setPopTime(popTime); + record.setInvisibleTime(invisibleTime); + record.setTopicId("topic"); + record.setGroupId("group"); + record.setQueueId(0); + record.setOffset(0); + consumerService.getPopConsumerStore().writeRecords(Collections.singletonList(record)); + + Mockito.doReturn(CompletableFuture.completedFuture(Triple.of(Mockito.mock(MessageExt.class), "", false))) + .when(consumerServiceSpy).getMessageAsync(any(PopConsumerRecord.class)); + Mockito.when(brokerController.getEscapeBridge().putMessageToSpecificQueue(any(MessageExtBrokerInner.class))).thenReturn( + new PutMessageResult(PutMessageStatus.UNKNOWN_ERROR, new AppendMessageResult(AppendMessageStatus.UNKNOWN_ERROR)) + ); + + long visibleTimestamp = popTime + invisibleTime; + + // revive fails + Assert.assertEquals(1, consumerServiceSpy.revive(new AtomicLong(visibleTimestamp), 1)); + // should be invisible now + Assert.assertEquals(0, consumerService.getPopConsumerStore().scanExpiredRecords(0, visibleTimestamp, 1).size()); + // will be visible again in 10 seconds + Assert.assertEquals(1, consumerService.getPopConsumerStore().scanExpiredRecords(visibleTimestamp, System.currentTimeMillis() + visibleTimestamp + 10 * 1000, 1).size()); + + consumerService.shutdown(); + } + + @Test + public void transferToFsStoreTest() { + Assert.assertNotNull(consumerService.getServiceName()); + List consumerRecordList = IntStream.range(0, 3) + .mapToObj(i -> { + PopConsumerRecord temp = new PopConsumerRecord(); + temp.setPopTime(0); + temp.setInvisibleTime(20 * 1000); + temp.setTopicId("topic"); + temp.setGroupId("group"); + temp.setQueueId(2); + temp.setOffset(i); + return temp; + }) + .collect(Collectors.toList()); + + Mockito.when(brokerController.getPopMessageProcessor().buildCkMsg(any(), anyInt())) + .thenReturn(new MessageExtBrokerInner()); + Mockito.when(brokerController.getMessageStore()).thenReturn(Mockito.mock(MessageStore.class)); + Mockito.when(brokerController.getMessageStore().asyncPutMessage(any())) + .thenReturn(CompletableFuture.completedFuture( + new PutMessageResult(PutMessageStatus.PUT_OK, new AppendMessageResult(AppendMessageStatus.PUT_OK)))); + + consumerService.start(); + consumerService.getPopConsumerStore().writeRecords(consumerRecordList); + consumerService.transferToFsStore(); + consumerService.shutdown(); + } +} \ No newline at end of file diff --git a/broker/src/test/java/org/apache/rocketmq/broker/processor/RecallMessageProcessorTest.java b/broker/src/test/java/org/apache/rocketmq/broker/processor/RecallMessageProcessorTest.java index 7bd260cc2c0..d28eb2f1dff 100644 --- a/broker/src/test/java/org/apache/rocketmq/broker/processor/RecallMessageProcessorTest.java +++ b/broker/src/test/java/org/apache/rocketmq/broker/processor/RecallMessageProcessorTest.java @@ -89,6 +89,7 @@ public void init() throws IllegalAccessException, NoSuchFieldException { when(brokerController.getMessageStore()).thenReturn(messageStore); when(brokerController.getBrokerConfig()).thenReturn(brokerConfig); when(brokerConfig.getBrokerName()).thenReturn(BROKER_NAME); + when(brokerConfig.isRecallMessageEnable()).thenReturn(true); when(brokerController.getBrokerStatsManager()).thenReturn(brokerStatsManager); when(handlerContext.channel()).thenReturn(channel); recallMessageProcessor = new RecallMessageProcessor(brokerController); @@ -134,6 +135,14 @@ public void testHandlePutMessageResult() { } } + @Test + public void testProcessRequest_notEnable() throws RemotingCommandException { + when(brokerConfig.isRecallMessageEnable()).thenReturn(false); + RemotingCommand request = mockRequest(0, TOPIC, TOPIC, "id", BROKER_NAME); + RemotingCommand response = recallMessageProcessor.processRequest(handlerContext, request); + Assert.assertEquals(ResponseCode.NO_PERMISSION, response.getCode()); + } + @Test public void testProcessRequest_invalidStatus() throws RemotingCommandException { RemotingCommand request = mockRequest(0, TOPIC, TOPIC, "id", BROKER_NAME); diff --git a/broker/src/test/java/org/apache/rocketmq/broker/slave/SlaveSynchronizeAtomicTest.java b/broker/src/test/java/org/apache/rocketmq/broker/slave/SlaveSynchronizeAtomicTest.java new file mode 100644 index 00000000000..75db22e7e77 --- /dev/null +++ b/broker/src/test/java/org/apache/rocketmq/broker/slave/SlaveSynchronizeAtomicTest.java @@ -0,0 +1,141 @@ +/* + * 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.rocketmq.broker.slave; + +import static org.mockito.ArgumentMatchers.anyString; +import static org.mockito.Mockito.when; +import java.util.concurrent.ConcurrentHashMap; +import java.util.concurrent.CountDownLatch; +import java.util.concurrent.ThreadLocalRandom; +import org.apache.rocketmq.broker.BrokerController; +import org.apache.rocketmq.broker.loadbalance.MessageRequestModeManager; +import org.apache.rocketmq.broker.out.BrokerOuterAPI; +import org.apache.rocketmq.broker.processor.QueryAssignmentProcessor; +import org.apache.rocketmq.broker.subscription.SubscriptionGroupManager; +import org.apache.rocketmq.broker.topic.TopicConfigManager; +import org.apache.rocketmq.client.exception.MQBrokerException; +import org.apache.rocketmq.common.BrokerConfig; +import org.apache.rocketmq.remoting.exception.RemotingConnectException; +import org.apache.rocketmq.remoting.exception.RemotingSendRequestException; +import org.apache.rocketmq.remoting.exception.RemotingTimeoutException; +import org.apache.rocketmq.remoting.netty.NettyClientConfig; +import org.apache.rocketmq.remoting.netty.NettyServerConfig; +import org.apache.rocketmq.remoting.protocol.DataVersion; +import org.apache.rocketmq.remoting.protocol.body.MessageRequestModeSerializeWrapper; +import org.apache.rocketmq.remoting.protocol.body.SubscriptionGroupWrapper; +import org.apache.rocketmq.remoting.protocol.subscription.SubscriptionGroupConfig; +import org.apache.rocketmq.store.config.MessageStoreConfig; +import org.junit.Assert; +import org.junit.Before; +import org.junit.Test; +import org.junit.runner.RunWith; +import org.mockito.Mock; +import org.mockito.Spy; +import org.mockito.junit.MockitoJUnitRunner; + +@RunWith(MockitoJUnitRunner.class) +public class SlaveSynchronizeAtomicTest { + @Spy + private BrokerController brokerController = + new BrokerController(new BrokerConfig(), new NettyServerConfig(), new NettyClientConfig(), + new MessageStoreConfig()); + + private SlaveSynchronize slaveSynchronize; + + @Mock + private BrokerOuterAPI brokerOuterAPI; + + @Mock + private TopicConfigManager topicConfigManager; + + + @Mock + private SubscriptionGroupManager subscriptionGroupManager; + + @Mock + private QueryAssignmentProcessor queryAssignmentProcessor; + + @Mock + private MessageRequestModeManager messageRequestModeManager; + + + private static final String BROKER_ADDR = "127.0.0.1:10911"; + private final SubscriptionGroupWrapper subscriptionGroupWrapper = createSubscriptionGroupWrapper(); + private final MessageRequestModeSerializeWrapper requestModeSerializeWrapper = createMessageRequestModeWrapper(); + private final DataVersion dataVersion = new DataVersion(); + + @Before + public void init() { + for (int i = 0; i < 100000; i++) { + subscriptionGroupWrapper.getSubscriptionGroupTable().put("group" + i, new SubscriptionGroupConfig()); + } + for (int i = 0; i < 100000; i++) { + requestModeSerializeWrapper.getMessageRequestModeMap().put("topic" + i, new ConcurrentHashMap<>()); + } + when(brokerController.getBrokerOuterAPI()).thenReturn(brokerOuterAPI); + when(brokerController.getTopicConfigManager()).thenReturn(topicConfigManager); + when(brokerController.getSubscriptionGroupManager()).thenReturn(subscriptionGroupManager); + when(subscriptionGroupManager.getDataVersion()).thenReturn(dataVersion); + when(subscriptionGroupManager.getSubscriptionGroupTable()).thenReturn( + subscriptionGroupWrapper.getSubscriptionGroupTable()); + slaveSynchronize = new SlaveSynchronize(brokerController); + slaveSynchronize.setMasterAddr(BROKER_ADDR); + } + + private SubscriptionGroupWrapper createSubscriptionGroupWrapper() { + SubscriptionGroupWrapper wrapper = new SubscriptionGroupWrapper(); + wrapper.setSubscriptionGroupTable(new ConcurrentHashMap<>()); + DataVersion dataVersion = new DataVersion(); + dataVersion.setStateVersion(1L); + wrapper.setDataVersion(dataVersion); + return wrapper; + } + + private MessageRequestModeSerializeWrapper createMessageRequestModeWrapper() { + MessageRequestModeSerializeWrapper wrapper = new MessageRequestModeSerializeWrapper(); + wrapper.setMessageRequestModeMap(new ConcurrentHashMap<>()); + return wrapper; + } + + @Test + public void testSyncAtomically() + throws RemotingConnectException, RemotingSendRequestException, RemotingTimeoutException, MQBrokerException, + InterruptedException { + when(brokerOuterAPI.getAllSubscriptionGroupConfig(anyString())).thenReturn(subscriptionGroupWrapper); + when(brokerOuterAPI.getAllMessageRequestMode(anyString())).thenReturn(requestModeSerializeWrapper); + + CountDownLatch countDownLatch = new CountDownLatch(1); + new Thread(() -> { + while (countDownLatch.getCount() > 0) { + dataVersion.nextVersion(); + try { + slaveSynchronize.syncAll(); + } catch (Exception e) { + e.printStackTrace(); + } + } + }).start(); + + for (int i = 0; i < 10000000; i++) { + Assert.assertTrue(subscriptionGroupWrapper.getSubscriptionGroupTable() + .containsKey("group" + ThreadLocalRandom.current().nextInt(0, 100000))); + Assert.assertTrue(requestModeSerializeWrapper.getMessageRequestModeMap() + .containsKey("topic" + ThreadLocalRandom.current().nextInt(0, 100000))); + } + countDownLatch.countDown(); + } +} diff --git a/broker/src/test/java/org/apache/rocketmq/broker/topic/TopicQueueMappingCleanServiceTest.java b/broker/src/test/java/org/apache/rocketmq/broker/topic/TopicQueueMappingCleanServiceTest.java new file mode 100644 index 00000000000..c7079c5248f --- /dev/null +++ b/broker/src/test/java/org/apache/rocketmq/broker/topic/TopicQueueMappingCleanServiceTest.java @@ -0,0 +1,169 @@ +/* + * 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.rocketmq.broker.topic; + +import org.apache.rocketmq.broker.BrokerController; +import org.apache.rocketmq.broker.out.BrokerOuterAPI; +import org.apache.rocketmq.common.BrokerConfig; +import org.apache.rocketmq.common.message.MessageQueue; +import org.apache.rocketmq.remoting.exception.RemotingException; +import org.apache.rocketmq.remoting.protocol.DataVersion; +import org.apache.rocketmq.remoting.protocol.admin.TopicOffset; +import org.apache.rocketmq.remoting.protocol.admin.TopicStatsTable; +import org.apache.rocketmq.remoting.protocol.route.TopicRouteData; +import org.apache.rocketmq.remoting.protocol.statictopic.LogicQueueMappingItem; +import org.apache.rocketmq.remoting.protocol.statictopic.TopicQueueMappingDetail; +import org.apache.rocketmq.remoting.rpc.RpcClient; +import org.apache.rocketmq.remoting.rpc.RpcRequest; +import org.apache.rocketmq.remoting.rpc.RpcResponse; +import org.apache.rocketmq.store.config.MessageStoreConfig; +import org.junit.Before; +import org.junit.Test; +import org.junit.runner.RunWith; +import org.mockito.Mock; +import org.mockito.junit.MockitoJUnitRunner; + +import java.util.Arrays; +import java.util.Collections; +import java.util.Map; +import java.util.concurrent.CompletableFuture; +import java.util.concurrent.ConcurrentHashMap; +import java.util.concurrent.ConcurrentMap; + +import static org.mockito.ArgumentMatchers.any; +import static org.mockito.ArgumentMatchers.anyBoolean; +import static org.mockito.ArgumentMatchers.anyLong; +import static org.mockito.ArgumentMatchers.anyString; +import static org.mockito.Mockito.mock; +import static org.mockito.Mockito.never; +import static org.mockito.Mockito.times; +import static org.mockito.Mockito.verify; +import static org.mockito.Mockito.when; + +@RunWith(MockitoJUnitRunner.class) +public class TopicQueueMappingCleanServiceTest { + + @Mock + private BrokerController brokerController; + + @Mock + private TopicQueueMappingManager topicQueueMappingManager; + + @Mock + private RpcClient rpcClient; + + @Mock + private MessageStoreConfig messageStoreConfig; + + @Mock + private BrokerConfig brokerConfig; + + @Mock + private BrokerOuterAPI brokerOuterAPI; + + private TopicQueueMappingCleanService topicQueueMappingCleanService; + + private final String defaultTopic = "defaultTopic"; + + private final String defaultBroker = "defaultBroker"; + + private final String deleteWhen = "00;01;02;03;04;05;06;07;08;09;10;11;12;13;14;15;16;17;18;19;20;21;22;23"; + + @Before + public void init() { + when(brokerOuterAPI.getRpcClient()).thenReturn(rpcClient); + when(brokerController.getBrokerOuterAPI()).thenReturn(brokerOuterAPI); + when(brokerController.getTopicQueueMappingManager()).thenReturn(topicQueueMappingManager); + when(brokerController.getMessageStoreConfig()).thenReturn(messageStoreConfig); + when(brokerController.getBrokerConfig()).thenReturn(brokerConfig); + topicQueueMappingCleanService = new TopicQueueMappingCleanService(brokerController); + } + + @Test + public void testCleanItemExpiredNoChange() throws Exception { + when(messageStoreConfig.getDeleteWhen()).thenReturn("04"); + topicQueueMappingCleanService.cleanItemExpired(); + verify(topicQueueMappingManager, never()).updateTopicQueueMapping(any(), anyBoolean(), anyBoolean(), anyBoolean()); + } + + @Test + public void testCleanItemExpiredWithChange() throws Exception { + when(messageStoreConfig.getDeleteWhen()).thenReturn(deleteWhen); + TopicQueueMappingDetail mappingDetail = new TopicQueueMappingDetail(defaultTopic, 2, defaultBroker, 1); + mappingDetail.getHostedQueues().put(0, + Arrays.asList(new LogicQueueMappingItem(0, 0, defaultBroker, 0, 0, 100, 0, 0), + new LogicQueueMappingItem(0, 1, defaultBroker, 1, 100, 200, 0, 0))); + when(topicQueueMappingManager.getTopicQueueMappingTable()).thenReturn(new ConcurrentHashMap<>(Collections.singletonMap(defaultTopic, mappingDetail))); + when(brokerConfig.getBrokerName()).thenReturn(defaultBroker); + TopicStatsTable topicStatsTable = mock(TopicStatsTable.class); + Map offsetTable = new ConcurrentHashMap<>(); + TopicOffset topicOffset = new TopicOffset(); + topicOffset.setMinOffset(0); + topicOffset.setMaxOffset(0); + MessageQueue messageQueue = new MessageQueue(defaultTopic, defaultBroker, 0); + offsetTable.put(messageQueue, topicOffset); + when(topicStatsTable.getOffsetTable()).thenReturn(offsetTable); + when(rpcClient.invoke(any(RpcRequest.class), anyLong())).thenReturn(CompletableFuture.completedFuture(new RpcResponse(0, null, topicStatsTable))); + DataVersion dataVersion = mock(DataVersion.class); + when(topicQueueMappingManager.getDataVersion()).thenReturn(dataVersion); + topicQueueMappingCleanService.cleanItemExpired(); + verify(topicQueueMappingManager, times(1)).updateTopicQueueMapping(any(), anyBoolean(), anyBoolean(), anyBoolean()); + } + + @Test + public void testCleanItemListMoreThanSecondGen() throws Exception { + when(brokerConfig.getBrokerName()).thenReturn(defaultBroker); + when(messageStoreConfig.getDeleteWhen()).thenReturn(deleteWhen); + TopicQueueMappingDetail mappingDetail = new TopicQueueMappingDetail(defaultTopic, 1, defaultBroker, 1); + mappingDetail.setHostedQueues(new ConcurrentHashMap<>()); + LogicQueueMappingItem logicQueueMappingItem = mock(LogicQueueMappingItem.class); + when(logicQueueMappingItem.getBname()).thenReturn("broker"); + mappingDetail.getHostedQueues().put(0, Collections.singletonList(logicQueueMappingItem)); + ConcurrentMap topicQueueMappingTable = new ConcurrentHashMap<>(); + topicQueueMappingTable.put(defaultBroker, mappingDetail); + when(topicQueueMappingManager.getTopicQueueMappingTable()).thenReturn(topicQueueMappingTable); + TopicRouteData topicRouteData = new TopicRouteData(); + when(brokerOuterAPI.getTopicRouteInfoFromNameServer(any(), anyLong())).thenReturn(topicRouteData); + topicQueueMappingCleanService.cleanItemListMoreThanSecondGen(); + verify(brokerOuterAPI, times(1)).getTopicRouteInfoFromNameServer(any(), anyLong()); + } + + @Test + public void testCleanItemListMoreThanSecondGenNoChange() throws Exception { + when(messageStoreConfig.getDeleteWhen()).thenReturn("04"); + topicQueueMappingCleanService.cleanItemListMoreThanSecondGen(); + verify(brokerOuterAPI, never()).getTopicRouteInfoFromNameServer(anyString(), anyLong()); + verify(rpcClient, never()).invoke(any(RpcRequest.class), anyLong()); + } + + @Test + public void testCleanItemListMoreThanSecondGenException() throws Exception { + when(brokerConfig.getBrokerName()).thenReturn(defaultBroker); + when(messageStoreConfig.getDeleteWhen()).thenReturn(deleteWhen); + TopicQueueMappingDetail mappingDetail = new TopicQueueMappingDetail(defaultTopic, 1, defaultBroker, 1); + mappingDetail.setHostedQueues(new ConcurrentHashMap<>()); + LogicQueueMappingItem logicQueueMappingItem = mock(LogicQueueMappingItem.class); + when(logicQueueMappingItem.getBname()).thenReturn("broker"); + mappingDetail.getHostedQueues().put(0, Collections.singletonList(logicQueueMappingItem)); + ConcurrentMap topicQueueMappingTable = new ConcurrentHashMap<>(); + topicQueueMappingTable.put(defaultBroker, mappingDetail); + when(topicQueueMappingManager.getTopicQueueMappingTable()).thenReturn(topicQueueMappingTable); + when(brokerOuterAPI.getTopicRouteInfoFromNameServer(any(), anyLong())).thenThrow(new RemotingException("Test exception")); + topicQueueMappingCleanService.cleanItemListMoreThanSecondGen(); + verify(brokerOuterAPI, times(1)).getTopicRouteInfoFromNameServer(any(), anyLong()); + } +} diff --git a/broker/src/test/resources/rmq.logback-test.xml b/broker/src/test/resources/rmq.logback-test.xml index 8695d52d57c..7a2ff0bc933 100644 --- a/broker/src/test/resources/rmq.logback-test.xml +++ b/broker/src/test/resources/rmq.logback-test.xml @@ -19,9 +19,7 @@ - - %d{HH:mm:ss.SSS} [%t] %-5level %logger{36} - %msg%n - + ${CONSOLE_LOG_PATTERN} @@ -29,7 +27,10 @@ - + + + diff --git a/client/src/main/java/org/apache/rocketmq/client/impl/MQClientAPIImpl.java b/client/src/main/java/org/apache/rocketmq/client/impl/MQClientAPIImpl.java index 2e088ac9da5..114093e3502 100644 --- a/client/src/main/java/org/apache/rocketmq/client/impl/MQClientAPIImpl.java +++ b/client/src/main/java/org/apache/rocketmq/client/impl/MQClientAPIImpl.java @@ -164,6 +164,7 @@ import org.apache.rocketmq.remoting.protocol.header.DeleteTopicRequestHeader; import org.apache.rocketmq.remoting.protocol.header.DeleteUserRequestHeader; import org.apache.rocketmq.remoting.protocol.header.EndTransactionRequestHeader; +import org.apache.rocketmq.remoting.protocol.header.ExportRocksDBConfigToJsonRequestHeader; import org.apache.rocketmq.remoting.protocol.header.ExtraInfoUtil; import org.apache.rocketmq.remoting.protocol.header.GetAclRequestHeader; import org.apache.rocketmq.remoting.protocol.header.GetAllProducerInfoRequestHeader; @@ -776,7 +777,7 @@ private void onExceptionImpl(final String brokerName, final DefaultMQProducerImpl producer ) { int tmp = curTimes.incrementAndGet(); - if (needRetry && tmp <= timesTotal) { + if (needRetry && tmp <= timesTotal && timeoutMillis > 0) { String retryBrokerName = brokerName;//by default, it will send to the same broker if (topicPublishInfo != null) { //select one message queue accordingly, in order to determine which broker to send MessageQueue mqChosen = producer.selectOneMessageQueue(topicPublishInfo, brokerName, false); @@ -3036,6 +3037,21 @@ public CheckRocksdbCqWriteResult checkRocksdbCqWriteProgress(final String broker throw new MQClientException(response.getCode(), response.getRemark()); } + public void exportRocksDBConfigToJson(final String brokerAddr, + final List configType, + final long timeoutMillis) throws InterruptedException, + RemotingTimeoutException, RemotingSendRequestException, RemotingConnectException, MQClientException { + ExportRocksDBConfigToJsonRequestHeader header = new ExportRocksDBConfigToJsonRequestHeader(); + header.updateConfigType(configType); + RemotingCommand request = RemotingCommand.createRequestCommand(RequestCode.EXPORT_ROCKSDB_CONFIG_TO_JSON, header); + RemotingCommand response = this.remotingClient.invokeSync(brokerAddr, request, timeoutMillis); + assert response != null; + + if (ResponseCode.SUCCESS != response.getCode()) { + throw new MQClientException(response.getCode(), response.getRemark()); + } + } + public void checkClientInBroker(final String brokerAddr, final String consumerGroup, final String clientId, final SubscriptionData subscriptionData, final long timeoutMillis) @@ -3573,4 +3589,16 @@ public void operationFail(Throwable throwable) { } }); } + + public void exportPopRecord(String brokerAddr, long timeout) throws RemotingConnectException, + RemotingSendRequestException, RemotingTimeoutException, InterruptedException, MQBrokerException { + RemotingCommand request = RemotingCommand.createRequestCommand( + RequestCode.POP_ROLLBACK, null); + RemotingCommand response = this.remotingClient.invokeSync(brokerAddr, request, timeout); + assert response != null; + if (response.getCode() == SUCCESS) { + return; + } + throw new MQBrokerException(response.getCode(), response.getRemark()); + } } diff --git a/client/src/main/java/org/apache/rocketmq/client/impl/consumer/RebalanceImpl.java b/client/src/main/java/org/apache/rocketmq/client/impl/consumer/RebalanceImpl.java index d1f0d116e05..b6f1d99b1c7 100644 --- a/client/src/main/java/org/apache/rocketmq/client/impl/consumer/RebalanceImpl.java +++ b/client/src/main/java/org/apache/rocketmq/client/impl/consumer/RebalanceImpl.java @@ -36,7 +36,6 @@ import org.apache.rocketmq.common.message.MessageQueue; import org.apache.rocketmq.common.message.MessageQueueAssignment; import org.apache.rocketmq.common.message.MessageRequestMode; -import org.apache.rocketmq.remoting.exception.RemotingTimeoutException; import org.apache.rocketmq.remoting.protocol.body.LockBatchRequestBody; import org.apache.rocketmq.remoting.protocol.body.UnlockBatchRequestBody; import org.apache.rocketmq.remoting.protocol.filter.FilterAPI; @@ -60,12 +59,8 @@ public abstract class RebalanceImpl { protected MessageModel messageModel; protected AllocateMessageQueueStrategy allocateMessageQueueStrategy; protected MQClientInstance mQClientFactory; - private static final int TIMEOUT_CHECK_TIMES = 3; private static final int QUERY_ASSIGNMENT_TIMEOUT = 3000; - private Map topicBrokerRebalance = new ConcurrentHashMap<>(); - private Map topicClientRebalance = new ConcurrentHashMap<>(); - public RebalanceImpl(String consumerGroup, MessageModel messageModel, AllocateMessageQueueStrategy allocateMessageQueueStrategy, MQClientInstance mQClientFactory) { @@ -241,7 +236,7 @@ public boolean doRebalance(final boolean isOrder) { for (final Map.Entry entry : subTable.entrySet()) { final String topic = entry.getKey(); try { - if (!clientRebalance(topic) && tryQueryAssignment(topic)) { + if (!clientRebalance(topic)) { boolean result = this.getRebalanceResultFromBroker(topic, isOrder); if (!result) { balanced = false; @@ -266,38 +261,6 @@ public boolean doRebalance(final boolean isOrder) { return balanced; } - private boolean tryQueryAssignment(String topic) { - if (topicClientRebalance.containsKey(topic)) { - return false; - } - - if (topicBrokerRebalance.containsKey(topic)) { - return true; - } - String strategyName = allocateMessageQueueStrategy != null ? allocateMessageQueueStrategy.getName() : null; - int retryTimes = 0; - while (retryTimes++ < TIMEOUT_CHECK_TIMES) { - try { - Set resultSet = mQClientFactory.queryAssignment(topic, consumerGroup, - strategyName, messageModel, QUERY_ASSIGNMENT_TIMEOUT / TIMEOUT_CHECK_TIMES * retryTimes); - topicBrokerRebalance.put(topic, topic); - return true; - } catch (Throwable t) { - if (!(t instanceof RemotingTimeoutException)) { - log.error("tryQueryAssignment error.", t); - topicClientRebalance.put(topic, topic); - return false; - } - } - } - if (retryTimes >= TIMEOUT_CHECK_TIMES) { - // if never success before and timeout exceed TIMEOUT_CHECK_TIMES, force client rebalance - topicClientRebalance.put(topic, topic); - return false; - } - return true; - } - public ConcurrentMap getSubscriptionInner() { return subscriptionInner; } @@ -460,20 +423,6 @@ private void truncateMessageQueueNotMyTopic() { } } } - - Iterator> clientIter = topicClientRebalance.entrySet().iterator(); - while (clientIter.hasNext()) { - if (!subTable.containsKey(clientIter.next().getKey())) { - clientIter.remove(); - } - } - - Iterator> brokerIter = topicBrokerRebalance.entrySet().iterator(); - while (brokerIter.hasNext()) { - if (!subTable.containsKey(brokerIter.next().getKey())) { - brokerIter.remove(); - } - } } private boolean updateProcessQueueTableInRebalance(final String topic, final Set mqSet, diff --git a/common/src/main/java/org/apache/rocketmq/common/BrokerConfig.java b/common/src/main/java/org/apache/rocketmq/common/BrokerConfig.java index bac2e2c7e40..dd345449351 100644 --- a/common/src/main/java/org/apache/rocketmq/common/BrokerConfig.java +++ b/common/src/main/java/org/apache/rocketmq/common/BrokerConfig.java @@ -236,6 +236,13 @@ public class BrokerConfig extends BrokerIdentity { private boolean retrieveMessageFromPopRetryTopicV1 = true; private boolean enableRetryTopicV2 = false; private int popFromRetryProbability = 20; + private boolean popConsumerFSServiceInit = true; + private boolean popConsumerKVServiceLog = false; + private boolean popConsumerKVServiceInit = false; + private boolean popConsumerKVServiceEnable = false; + private int popReviveMaxReturnSizePerRead = 16 * 1024; + private int popReviveMaxAttemptTimes = 16; + private boolean realTimeNotifyConsumerChange = true; private boolean litePullMessageEnable = true; @@ -446,6 +453,8 @@ public class BrokerConfig extends BrokerIdentity { private boolean allowRecallWhenBrokerNotWriteable = true; + private boolean recallMessageEnable = false; + public String getConfigBlackList() { return configBlackList; } @@ -590,6 +599,53 @@ public void setPopFromRetryProbability(int popFromRetryProbability) { this.popFromRetryProbability = popFromRetryProbability; } + public boolean isPopConsumerFSServiceInit() { + return popConsumerFSServiceInit; + } + + public void setPopConsumerFSServiceInit(boolean popConsumerFSServiceInit) { + this.popConsumerFSServiceInit = popConsumerFSServiceInit; + } + + public boolean isPopConsumerKVServiceLog() { + return popConsumerKVServiceLog; + } + + public void setPopConsumerKVServiceLog(boolean popConsumerKVServiceLog) { + this.popConsumerKVServiceLog = popConsumerKVServiceLog; + } + + public boolean isPopConsumerKVServiceInit() { + return popConsumerKVServiceInit; + } + + public void setPopConsumerKVServiceInit(boolean popConsumerKVServiceInit) { + this.popConsumerKVServiceInit = popConsumerKVServiceInit; + } + + public boolean isPopConsumerKVServiceEnable() { + return popConsumerKVServiceEnable; + } + + public void setPopConsumerKVServiceEnable(boolean popConsumerKVServiceEnable) { + this.popConsumerKVServiceEnable = popConsumerKVServiceEnable; + } + + public int getPopReviveMaxReturnSizePerRead() { + return popReviveMaxReturnSizePerRead; + } + + public void setPopReviveMaxReturnSizePerRead(int popReviveMaxReturnSizePerRead) { + this.popReviveMaxReturnSizePerRead = popReviveMaxReturnSizePerRead; + } + + public int getPopReviveMaxAttemptTimes() { + return popReviveMaxAttemptTimes; + } + + public void setPopReviveMaxAttemptTimes(int popReviveMaxAttemptTimes) { + this.popReviveMaxAttemptTimes = popReviveMaxAttemptTimes; + } public boolean isTraceOn() { return traceOn; @@ -1942,4 +1998,12 @@ public boolean isAllowRecallWhenBrokerNotWriteable() { public void setAllowRecallWhenBrokerNotWriteable(boolean allowRecallWhenBrokerNotWriteable) { this.allowRecallWhenBrokerNotWriteable = allowRecallWhenBrokerNotWriteable; } + + public boolean isRecallMessageEnable() { + return recallMessageEnable; + } + + public void setRecallMessageEnable(boolean recallMessageEnable) { + this.recallMessageEnable = recallMessageEnable; + } } diff --git a/common/src/main/java/org/apache/rocketmq/common/config/AbstractRocksDBStorage.java b/common/src/main/java/org/apache/rocketmq/common/config/AbstractRocksDBStorage.java index 48ba4b8086c..6c0bce5929a 100644 --- a/common/src/main/java/org/apache/rocketmq/common/config/AbstractRocksDBStorage.java +++ b/common/src/main/java/org/apache/rocketmq/common/config/AbstractRocksDBStorage.java @@ -18,7 +18,16 @@ import com.google.common.collect.Maps; import io.netty.buffer.PooledByteBufAllocator; +import java.nio.ByteBuffer; import java.nio.charset.StandardCharsets; +import java.util.ArrayList; +import java.util.List; +import java.util.Map; +import java.util.concurrent.ArrayBlockingQueue; +import java.util.concurrent.ScheduledExecutorService; +import java.util.concurrent.Semaphore; +import java.util.concurrent.ThreadPoolExecutor; +import java.util.concurrent.TimeUnit; import org.apache.rocketmq.common.ThreadFactoryImpl; import org.apache.rocketmq.common.constant.LoggerName; import org.apache.rocketmq.common.utils.ThreadUtils; @@ -43,16 +52,6 @@ import org.rocksdb.WriteBatch; import org.rocksdb.WriteOptions; -import java.nio.ByteBuffer; -import java.util.ArrayList; -import java.util.List; -import java.util.Map; -import java.util.concurrent.ArrayBlockingQueue; -import java.util.concurrent.ScheduledExecutorService; -import java.util.concurrent.Semaphore; -import java.util.concurrent.ThreadPoolExecutor; -import java.util.concurrent.TimeUnit; - public abstract class AbstractRocksDBStorage { protected static final Logger LOGGER = LoggerFactory.getLogger(LoggerName.ROCKSDB_LOGGER_NAME); @@ -365,7 +364,7 @@ public synchronized boolean start() { } if (postLoad()) { this.loaded = true; - LOGGER.info("RocksDB[{}] starts OK", this.dbPath); + LOGGER.info("RocksDB [{}] starts OK", this.dbPath); this.closed = false; return true; } else { @@ -381,6 +380,7 @@ public synchronized boolean start() { public synchronized boolean shutdown() { try { if (!this.loaded) { + LOGGER.info("RocksDBStorage is not loaded, shutdown OK. dbPath={}, readOnly={}", this.dbPath, this.readOnly); return true; } @@ -437,9 +437,9 @@ public synchronized boolean shutdown() { this.options = null; this.loaded = false; - LOGGER.info("shutdown OK. {}", this.dbPath); + LOGGER.info("RocksDB shutdown OK. {}", this.dbPath); } catch (Exception e) { - LOGGER.error("shutdown Failed. {}", this.dbPath, e); + LOGGER.error("RocksDB shutdown failed. {}", this.dbPath, e); return false; } return true; diff --git a/common/src/main/java/org/apache/rocketmq/common/message/Message.java b/common/src/main/java/org/apache/rocketmq/common/message/Message.java index c7997c47318..acd4df96d28 100644 --- a/common/src/main/java/org/apache/rocketmq/common/message/Message.java +++ b/common/src/main/java/org/apache/rocketmq/common/message/Message.java @@ -108,6 +108,13 @@ public String getProperty(final String name) { return this.properties.get(name); } + public boolean hasProperty(final String name) { + if (null == this.properties) { + return false; + } + return this.properties.containsKey(name); + } + public String getTopic() { return topic; } diff --git a/docs/cn/Deployment.md b/docs/cn/Deployment.md index 14529d111b0..c13f3280350 100644 --- a/docs/cn/Deployment.md +++ b/docs/cn/Deployment.md @@ -69,7 +69,7 @@ $ nohup sh mqbroker -n 192.168.1.1:9876 -c $ROCKETMQ_HOME/conf/2m-noslave/broker ... ``` -上面显示的启动命令用于单个NameServer的情况。对于多个NameServer的集群,broker 启动命令中-n参数后面的地址列表用分号隔开,例如 192.168.1.1:9876;192.161.2:9876 +上面显示的启动命令用于单个NameServer的情况。对于多个NameServer的集群,broker 启动命令中-n参数后面的地址列表用分号隔开,例如 192.168.1.1:9876;192.168.1.2:9876 ### 3 多Master多Slave模式-异步复制 @@ -168,4 +168,3 @@ RocketMQ 5.0 开始支持自动主从切换的模式,可参考以下文档 [设计思想](controller/design.md) - diff --git a/docs/en/Deployment.md b/docs/en/Deployment.md index 5dc93488a77..8f5c8f1e388 100644 --- a/docs/en/Deployment.md +++ b/docs/en/Deployment.md @@ -69,7 +69,7 @@ $ nohup sh mqbroker -n 192.168.1.1:9876 -c $ROCKETMQ_HOME/conf/2m-noslave/broker ... ``` -The boot command shown above is used in the case of a single NameServer.For clusters of multiple NameServer, the address list after the -n argument in the broker boot command is separated by semicolons, for example, 192.168.1.1: 9876;192.161.2: 9876. +The boot command shown above is used in the case of a single NameServer.For clusters of multiple NameServer, the address list after the -n argument in the broker boot command is separated by semicolons, for example, 192.168.1.1: 9876;192.168.1.2: 9876. ### 3 Multiple Master And Multiple Slave Mode-Asynchronous replication diff --git a/proxy/src/main/java/org/apache/rocketmq/proxy/grpc/v2/consumer/AckMessageActivity.java b/proxy/src/main/java/org/apache/rocketmq/proxy/grpc/v2/consumer/AckMessageActivity.java index 4a5b9cfcd62..76019a1ca94 100644 --- a/proxy/src/main/java/org/apache/rocketmq/proxy/grpc/v2/consumer/AckMessageActivity.java +++ b/proxy/src/main/java/org/apache/rocketmq/proxy/grpc/v2/consumer/AckMessageActivity.java @@ -34,6 +34,7 @@ import org.apache.rocketmq.proxy.config.ConfigurationManager; import org.apache.rocketmq.proxy.grpc.v2.AbstractMessingActivity; import org.apache.rocketmq.proxy.grpc.v2.channel.GrpcChannelManager; +import org.apache.rocketmq.proxy.grpc.v2.channel.GrpcClientChannel; import org.apache.rocketmq.proxy.grpc.v2.common.GrpcClientSettingsManager; import org.apache.rocketmq.proxy.grpc.v2.common.ResponseBuilder; import org.apache.rocketmq.proxy.processor.BatchAckResult; @@ -193,10 +194,12 @@ protected void setAckResponseStatus(AckMessageResponse.Builder responseBuilder, protected String getHandleString(ProxyContext ctx, String group, AckMessageRequest request, AckMessageEntry ackMessageEntry) { String handleString = ackMessageEntry.getReceiptHandle(); - - MessageReceiptHandle messageReceiptHandle = messagingProcessor.removeReceiptHandle(ctx, grpcChannelManager.getChannel(ctx.getClientID()), group, ackMessageEntry.getMessageId(), ackMessageEntry.getReceiptHandle()); - if (messageReceiptHandle != null) { - handleString = messageReceiptHandle.getReceiptHandleStr(); + GrpcClientChannel channel = grpcChannelManager.getChannel(ctx.getClientID()); + if (channel != null) { + MessageReceiptHandle messageReceiptHandle = messagingProcessor.removeReceiptHandle(ctx, channel, group, ackMessageEntry.getMessageId(), ackMessageEntry.getReceiptHandle()); + if (messageReceiptHandle != null) { + handleString = messageReceiptHandle.getReceiptHandleStr(); + } } return handleString; } diff --git a/proxy/src/main/java/org/apache/rocketmq/proxy/processor/ProducerProcessor.java b/proxy/src/main/java/org/apache/rocketmq/proxy/processor/ProducerProcessor.java index 43e16ddd2d7..17a2f27fa74 100644 --- a/proxy/src/main/java/org/apache/rocketmq/proxy/processor/ProducerProcessor.java +++ b/proxy/src/main/java/org/apache/rocketmq/proxy/processor/ProducerProcessor.java @@ -74,7 +74,7 @@ public CompletableFuture> sendMessage(ProxyContext ctx, QueueSe try { Message message = messageList.get(0); String topic = message.getTopic(); - if (ConfigurationManager.getProxyConfig().isEnableTopicMessageTypeCheck()) { + if (isNeedCheckTopicMessageType(message)) { if (topicMessageTypeValidator != null) { // Do not check retry or dlq topic if (!NamespaceUtil.isRetryTopic(topic) && !NamespaceUtil.isDLQTopic(topic)) { @@ -261,4 +261,8 @@ public CompletableFuture forwardMessageToDeadLetterQueue(ProxyC return FutureUtils.addExecutor(future, this.executor); } + private boolean isNeedCheckTopicMessageType(Message message) { + return ConfigurationManager.getProxyConfig().isEnableTopicMessageTypeCheck() + && !message.hasProperty(MessageConst.PROPERTY_TRANSFER_FLAG); + } } diff --git a/proxy/src/main/java/org/apache/rocketmq/proxy/remoting/activity/SendMessageActivity.java b/proxy/src/main/java/org/apache/rocketmq/proxy/remoting/activity/SendMessageActivity.java index 17af0fdcb37..22d9efd9347 100644 --- a/proxy/src/main/java/org/apache/rocketmq/proxy/remoting/activity/SendMessageActivity.java +++ b/proxy/src/main/java/org/apache/rocketmq/proxy/remoting/activity/SendMessageActivity.java @@ -21,17 +21,18 @@ import java.time.Duration; import java.util.Map; import org.apache.rocketmq.common.attribute.TopicMessageType; +import org.apache.rocketmq.common.message.MessageConst; import org.apache.rocketmq.common.message.MessageDecoder; -import org.apache.rocketmq.remoting.protocol.NamespaceUtil; -import org.apache.rocketmq.remoting.protocol.RequestCode; -import org.apache.rocketmq.remoting.protocol.header.SendMessageRequestHeader; import org.apache.rocketmq.proxy.common.ProxyContext; import org.apache.rocketmq.proxy.config.ConfigurationManager; import org.apache.rocketmq.proxy.processor.MessagingProcessor; import org.apache.rocketmq.proxy.processor.validator.DefaultTopicMessageTypeValidator; import org.apache.rocketmq.proxy.processor.validator.TopicMessageTypeValidator; import org.apache.rocketmq.proxy.remoting.pipeline.RequestPipeline; +import org.apache.rocketmq.remoting.protocol.NamespaceUtil; import org.apache.rocketmq.remoting.protocol.RemotingCommand; +import org.apache.rocketmq.remoting.protocol.RequestCode; +import org.apache.rocketmq.remoting.protocol.header.SendMessageRequestHeader; public class SendMessageActivity extends AbstractRemotingActivity { TopicMessageTypeValidator topicMessageTypeValidator; @@ -66,7 +67,7 @@ protected RemotingCommand sendMessage(ChannelHandlerContext ctx, RemotingCommand String topic = requestHeader.getTopic(); Map property = MessageDecoder.string2messageProperties(requestHeader.getProperties()); TopicMessageType messageType = TopicMessageType.parseFromMessageProperty(property); - if (ConfigurationManager.getProxyConfig().isEnableTopicMessageTypeCheck()) { + if (isNeedCheckTopicMessageType(property)) { if (topicMessageTypeValidator != null) { // Do not check retry or dlq topic if (!NamespaceUtil.isRetryTopic(topic) && !NamespaceUtil.isDLQTopic(topic)) { @@ -87,4 +88,9 @@ protected RemotingCommand consumerSendMessage(ChannelHandlerContext ctx, Remotin ProxyContext context) throws Exception { return request(ctx, request, context, Duration.ofSeconds(3).toMillis()); } + + private boolean isNeedCheckTopicMessageType(Map property) { + return ConfigurationManager.getProxyConfig().isEnableTopicMessageTypeCheck() + && !property.containsKey(MessageConst.PROPERTY_TRANSFER_FLAG); + } } diff --git a/remoting/src/main/java/org/apache/rocketmq/remoting/netty/NettyRemotingAbstract.java b/remoting/src/main/java/org/apache/rocketmq/remoting/netty/NettyRemotingAbstract.java index 3d4e62f9430..d3f5a88cf2a 100644 --- a/remoting/src/main/java/org/apache/rocketmq/remoting/netty/NettyRemotingAbstract.java +++ b/remoting/src/main/java/org/apache/rocketmq/remoting/netty/NettyRemotingAbstract.java @@ -49,6 +49,7 @@ import org.apache.rocketmq.common.ServiceThread; import org.apache.rocketmq.common.UtilAll; import org.apache.rocketmq.common.constant.LoggerName; +import org.apache.rocketmq.common.utils.ExceptionUtils; import org.apache.rocketmq.logging.org.slf4j.Logger; import org.apache.rocketmq.logging.org.slf4j.LoggerFactory; import org.apache.rocketmq.remoting.ChannelEventListener; @@ -602,7 +603,7 @@ public void invokeAsyncImpl(final Channel channel, final RemotingCommand request }) .thenAccept(responseFuture -> invokeCallback.operationSucceed(responseFuture.getResponseCommand())) .exceptionally(t -> { - invokeCallback.operationFail(t); + invokeCallback.operationFail(ExceptionUtils.getRealException(t)); return null; }); } diff --git a/remoting/src/main/java/org/apache/rocketmq/remoting/protocol/RequestCode.java b/remoting/src/main/java/org/apache/rocketmq/remoting/protocol/RequestCode.java index 9e86422c482..9cbbe834907 100644 --- a/remoting/src/main/java/org/apache/rocketmq/remoting/protocol/RequestCode.java +++ b/remoting/src/main/java/org/apache/rocketmq/remoting/protocol/RequestCode.java @@ -95,6 +95,7 @@ public class RequestCode { public static final int CHANGE_MESSAGE_INVISIBLETIME = 200053; public static final int NOTIFICATION = 200054; public static final int POLLING_INFO = 200055; + public static final int POP_ROLLBACK = 200056; public static final int PUT_KV_CONFIG = 100; @@ -212,12 +213,17 @@ public class RequestCode { public static final int PUSH_REPLY_MESSAGE_TO_CLIENT = 326; public static final int ADD_WRITE_PERM_OF_BROKER = 327; + + public static final int GET_ALL_PRODUCER_INFO = 328; + + public static final int DELETE_EXPIRED_COMMITLOG = 329; public static final int GET_TOPIC_CONFIG = 351; public static final int GET_SUBSCRIPTIONGROUP_CONFIG = 352; public static final int UPDATE_AND_GET_GROUP_FORBIDDEN = 353; public static final int CHECK_ROCKSDB_CQ_WRITE_PROGRESS = 354; + public static final int EXPORT_ROCKSDB_CONFIG_TO_JSON = 355; public static final int LITE_PULL_MESSAGE = 361; public static final int RECALL_MESSAGE = 370; @@ -244,10 +250,6 @@ public class RequestCode { public static final int RESET_MASTER_FLUSH_OFFSET = 908; - public static final int GET_ALL_PRODUCER_INFO = 328; - - public static final int DELETE_EXPIRED_COMMITLOG = 329; - /** * Controller code */ diff --git a/remoting/src/main/java/org/apache/rocketmq/remoting/protocol/header/ExportRocksDBConfigToJsonRequestHeader.java b/remoting/src/main/java/org/apache/rocketmq/remoting/protocol/header/ExportRocksDBConfigToJsonRequestHeader.java new file mode 100644 index 00000000000..7b1f9470e1e --- /dev/null +++ b/remoting/src/main/java/org/apache/rocketmq/remoting/protocol/header/ExportRocksDBConfigToJsonRequestHeader.java @@ -0,0 +1,100 @@ +/* + * 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.rocketmq.remoting.protocol.header; + +import java.util.ArrayList; +import java.util.List; +import org.apache.commons.lang3.StringUtils; +import org.apache.rocketmq.common.action.Action; +import org.apache.rocketmq.common.action.RocketMQAction; +import org.apache.rocketmq.remoting.CommandCustomHeader; +import org.apache.rocketmq.remoting.annotation.CFNotNull; +import org.apache.rocketmq.remoting.exception.RemotingCommandException; +import org.apache.rocketmq.remoting.protocol.RequestCode; + +@RocketMQAction(value = RequestCode.EXPORT_ROCKSDB_CONFIG_TO_JSON, action = Action.GET) +public class ExportRocksDBConfigToJsonRequestHeader implements CommandCustomHeader { + private static final String CONFIG_TYPE_SEPARATOR = ";"; + + public enum ConfigType { + TOPICS("topics"), + SUBSCRIPTION_GROUPS("subscriptionGroups"), + CONSUMER_OFFSETS("consumerOffsets"); + + private final String typeName; + + ConfigType(String typeName) { + this.typeName = typeName; + } + + public static ConfigType getConfigTypeByName(String typeName) { + for (ConfigType configType : ConfigType.values()) { + if (configType.getTypeName().equalsIgnoreCase(typeName.trim())) { + return configType; + } + } + throw new IllegalArgumentException("Unknown config type: " + typeName); + } + + public static List fromString(String ordinal) { + String[] configTypeNames = StringUtils.split(ordinal, CONFIG_TYPE_SEPARATOR); + List configTypes = new ArrayList<>(); + for (String configTypeName : configTypeNames) { + if (StringUtils.isNotEmpty(configTypeName)) { + configTypes.add(getConfigTypeByName(configTypeName)); + } + } + return configTypes; + } + + public static String toString(List configTypes) { + StringBuilder sb = new StringBuilder(); + for (ConfigType configType : configTypes) { + sb.append(configType.getTypeName()).append(CONFIG_TYPE_SEPARATOR); + } + return sb.toString(); + } + + public String getTypeName() { + return typeName; + } + } + + @CFNotNull + private String configType; + + @Override + public void checkFields() throws RemotingCommandException { + + } + + public List fetchConfigType() { + return ConfigType.fromString(configType); + } + + public void updateConfigType(List configType) { + this.configType = ConfigType.toString(configType); + } + + public String getConfigType() { + return configType; + } + + public void setConfigType(String configType) { + this.configType = configType; + } +} \ No newline at end of file diff --git a/remoting/src/test/java/org/apache/rocketmq/remoting/protocol/header/ExportRocksDBConfigToJsonRequestHeaderTest.java b/remoting/src/test/java/org/apache/rocketmq/remoting/protocol/header/ExportRocksDBConfigToJsonRequestHeaderTest.java new file mode 100644 index 00000000000..bbe625a42af --- /dev/null +++ b/remoting/src/test/java/org/apache/rocketmq/remoting/protocol/header/ExportRocksDBConfigToJsonRequestHeaderTest.java @@ -0,0 +1,51 @@ +/* + * 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.rocketmq.remoting.protocol.header; + +import java.util.ArrayList; +import java.util.List; +import org.junit.Assert; +import org.junit.Test; + +public class ExportRocksDBConfigToJsonRequestHeaderTest { + @Test + public void configTypeTest() { + List configTypes = new ArrayList<>(); + configTypes.add(ExportRocksDBConfigToJsonRequestHeader.ConfigType.TOPICS); + configTypes.add(ExportRocksDBConfigToJsonRequestHeader.ConfigType.SUBSCRIPTION_GROUPS); + + String string = ExportRocksDBConfigToJsonRequestHeader.ConfigType.toString(configTypes); + + List newConfigTypes = ExportRocksDBConfigToJsonRequestHeader.ConfigType.fromString(string); + assert newConfigTypes.size() == 2; + assert configTypes.equals(newConfigTypes); + + List topics = ExportRocksDBConfigToJsonRequestHeader.ConfigType.fromString("topics"); + assert topics.size() == 1; + assert topics.get(0).equals(ExportRocksDBConfigToJsonRequestHeader.ConfigType.TOPICS); + + List mix = ExportRocksDBConfigToJsonRequestHeader.ConfigType.fromString("toPics; subScriptiongroups"); + assert mix.size() == 2; + assert mix.get(0).equals(ExportRocksDBConfigToJsonRequestHeader.ConfigType.TOPICS); + assert mix.get(1).equals(ExportRocksDBConfigToJsonRequestHeader.ConfigType.SUBSCRIPTION_GROUPS); + + Assert.assertThrows(IllegalArgumentException.class, () -> { + ExportRocksDBConfigToJsonRequestHeader.ConfigType.fromString("topics; subscription"); + }); + + } +} diff --git a/store/src/main/java/org/apache/rocketmq/store/CommitLog.java b/store/src/main/java/org/apache/rocketmq/store/CommitLog.java index d30691908b2..b061aa7a0d4 100644 --- a/store/src/main/java/org/apache/rocketmq/store/CommitLog.java +++ b/store/src/main/java/org/apache/rocketmq/store/CommitLog.java @@ -432,8 +432,14 @@ private void doNothingForDeadCode(final Object obj) { public DispatchRequest checkMessageAndReturnSize(java.nio.ByteBuffer byteBuffer, final boolean checkCRC, final boolean checkDupInfo, final boolean readBody) { try { + if (byteBuffer.remaining() <= 4) { + return new DispatchRequest(-1, false /* fail */); + } // 1 TOTAL SIZE int totalSize = byteBuffer.getInt(); + if (byteBuffer.remaining() < totalSize - 4) { + return new DispatchRequest(-1, false /* fail */); + } // 2 MAGIC CODE int magicCode = byteBuffer.getInt(); @@ -577,7 +583,7 @@ public DispatchRequest checkMessageAndReturnSize(java.nio.ByteBuffer byteBuffer, } } } - if (expectedCRC > 0) { + if (expectedCRC >= 0) { ByteBuffer tmpBuffer = byteBuffer.duplicate(); tmpBuffer.position(tmpBuffer.position() - totalSize); tmpBuffer.limit(tmpBuffer.position() + totalSize - CommitLog.CRC32_RESERVED_LEN); @@ -628,6 +634,7 @@ public DispatchRequest checkMessageAndReturnSize(java.nio.ByteBuffer byteBuffer, return dispatchRequest; } catch (Exception e) { + log.error("checkMessageAndReturnSize failed, may can not dispatch", e); } return new DispatchRequest(-1, false /* success */); diff --git a/store/src/main/java/org/apache/rocketmq/store/ConsumeQueueExt.java b/store/src/main/java/org/apache/rocketmq/store/ConsumeQueueExt.java index 780505c53d1..3f266378df3 100644 --- a/store/src/main/java/org/apache/rocketmq/store/ConsumeQueueExt.java +++ b/store/src/main/java/org/apache/rocketmq/store/ConsumeQueueExt.java @@ -33,7 +33,7 @@ * such as message store time, filter bit map and etc. *

*

  • 1. This class is used only by {@link ConsumeQueue}
  • - *
  • 2. And is week reliable.
  • + *
  • 2. And is weakly reliable.
  • *
  • 3. Be careful, address returned is always less than 0.
  • *
  • 4. Pls keep this file small.
  • */ diff --git a/store/src/main/java/org/apache/rocketmq/store/DefaultMessageStore.java b/store/src/main/java/org/apache/rocketmq/store/DefaultMessageStore.java index 9d3c46a438a..187a0729e83 100644 --- a/store/src/main/java/org/apache/rocketmq/store/DefaultMessageStore.java +++ b/store/src/main/java/org/apache/rocketmq/store/DefaultMessageStore.java @@ -517,11 +517,9 @@ public void shutdown() { if (this.compactionService != null) { this.compactionService.shutdown(); } - - if (messageStoreConfig.isRocksdbCQDoubleWriteEnable()) { + if (messageStoreConfig.isRocksdbCQDoubleWriteEnable() && this.rocksDBMessageStore != null) { this.rocksDBMessageStore.consumeQueueStore.shutdown(); } - this.flushConsumeQueueService.shutdown(); this.allocateMappedFileService.shutdown(); this.storeCheckpoint.flush(); diff --git a/store/src/main/java/org/apache/rocketmq/store/queue/RocksDBConsumeQueueOffsetTable.java b/store/src/main/java/org/apache/rocketmq/store/queue/RocksDBConsumeQueueOffsetTable.java index a91ae5e244e..cb989852fb9 100644 --- a/store/src/main/java/org/apache/rocketmq/store/queue/RocksDBConsumeQueueOffsetTable.java +++ b/store/src/main/java/org/apache/rocketmq/store/queue/RocksDBConsumeQueueOffsetTable.java @@ -144,7 +144,7 @@ private void loadMaxConsumeQueueOffsets() { Function predicate = entry -> entry.type == OffsetEntryType.MAXIMUM; Consumer fn = entry -> { topicQueueMaxCqOffset.putIfAbsent(entry.topic + "-" + entry.queueId, entry.offset); - ROCKSDB_LOG.info("Max {}:{} --> {}|{}", entry.topic, entry.queueId, entry.offset, entry.commitLogOffset); + log.info("LoadMaxConsumeQueueOffsets Max {}:{} --> {}|{}", entry.topic, entry.queueId, entry.offset, entry.commitLogOffset); }; try { forEach(predicate, fn); diff --git a/store/src/main/java/org/apache/rocketmq/store/rocksdb/RocksDBOptionsFactory.java b/store/src/main/java/org/apache/rocketmq/store/rocksdb/RocksDBOptionsFactory.java index 2fac3bf485d..5687d6a222d 100644 --- a/store/src/main/java/org/apache/rocketmq/store/rocksdb/RocksDBOptionsFactory.java +++ b/store/src/main/java/org/apache/rocketmq/store/rocksdb/RocksDBOptionsFactory.java @@ -131,6 +131,57 @@ public static ColumnFamilyOptions createOffsetCFOptions() { setInplaceUpdateSupport(true); } + public static ColumnFamilyOptions createPopCFOptions() { + BlockBasedTableConfig blockBasedTableConfig = new BlockBasedTableConfig() + .setFormatVersion(5) + .setIndexType(IndexType.kBinarySearch) + .setDataBlockIndexType(DataBlockIndexType.kDataBlockBinaryAndHash) + .setDataBlockHashTableUtilRatio(0.75) + .setBlockSize(32 * SizeUnit.KB) + .setMetadataBlockSize(4 * SizeUnit.KB) + .setFilterPolicy(new BloomFilter(16, false)) + .setCacheIndexAndFilterBlocks(false) + .setCacheIndexAndFilterBlocksWithHighPriority(true) + .setPinL0FilterAndIndexBlocksInCache(false) + .setPinTopLevelIndexAndFilter(true) + .setBlockCache(new LRUCache(1024 * SizeUnit.MB, 8, false)) + .setWholeKeyFiltering(true); + + CompactionOptionsUniversal compactionOption = new CompactionOptionsUniversal() + .setSizeRatio(100) + .setMaxSizeAmplificationPercent(25) + .setAllowTrivialMove(true) + .setMinMergeWidth(2) + .setMaxMergeWidth(Integer.MAX_VALUE) + .setStopStyle(CompactionStopStyle.CompactionStopStyleTotalSize) + .setCompressionSizePercent(-1); + + //noinspection resource + return new ColumnFamilyOptions() + .setMaxWriteBufferNumber(4) + .setWriteBufferSize(128 * SizeUnit.MB) + .setMinWriteBufferNumberToMerge(1) + .setTableFormatConfig(blockBasedTableConfig) + .setMemTableConfig(new SkipListMemTableConfig()) + .setCompressionType(CompressionType.NO_COMPRESSION) + .setBottommostCompressionType(CompressionType.NO_COMPRESSION) + .setNumLevels(7) + .setCompactionPriority(CompactionPriority.MinOverlappingRatio) + .setCompactionStyle(CompactionStyle.UNIVERSAL) + .setCompactionOptionsUniversal(compactionOption) + .setMaxCompactionBytes(100 * SizeUnit.GB) + .setSoftPendingCompactionBytesLimit(100 * SizeUnit.GB) + .setHardPendingCompactionBytesLimit(256 * SizeUnit.GB) + .setLevel0FileNumCompactionTrigger(2) + .setLevel0SlowdownWritesTrigger(8) + .setLevel0StopWritesTrigger(10) + .setTargetFileSizeBase(256 * SizeUnit.MB) + .setTargetFileSizeMultiplier(2) + .setMergeOperator(new StringAppendOperator()) + .setReportBgIoStats(true) + .setOptimizeFiltersForHits(true); + } + /** * Create a rocksdb db options, the user must take care to close it after closing db. * @return diff --git a/test/src/test/java/org/apache/rocketmq/test/base/IntegrationTestBase.java b/test/src/test/java/org/apache/rocketmq/test/base/IntegrationTestBase.java index fde991ad13d..287e54d5617 100644 --- a/test/src/test/java/org/apache/rocketmq/test/base/IntegrationTestBase.java +++ b/test/src/test/java/org/apache/rocketmq/test/base/IntegrationTestBase.java @@ -138,6 +138,7 @@ public static BrokerController createAndStartBroker(String nsAddr) { brokerConfig.setEnableCalcFilterBitMap(true); brokerConfig.setAppendAckAsync(true); brokerConfig.setAppendCkAsync(true); + brokerConfig.setRecallMessageEnable(true); storeConfig.setEnableConsumeQueueExt(true); brokerConfig.setLoadBalancePollNameServerInterval(500); storeConfig.setStorePathRootDir(baseDir); diff --git a/tieredstore/src/main/java/org/apache/rocketmq/tieredstore/common/GroupCommitContext.java b/tieredstore/src/main/java/org/apache/rocketmq/tieredstore/common/GroupCommitContext.java new file mode 100644 index 00000000000..f677e7c934e --- /dev/null +++ b/tieredstore/src/main/java/org/apache/rocketmq/tieredstore/common/GroupCommitContext.java @@ -0,0 +1,70 @@ +/* + * 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.rocketmq.tieredstore.common; + +import java.util.List; +import org.apache.rocketmq.store.DispatchRequest; +import org.apache.rocketmq.store.SelectMappedBufferResult; + +public class GroupCommitContext { + + private long endOffset; + + private List bufferList; + + private List dispatchRequests; + + public long getEndOffset() { + return endOffset; + } + + public void setEndOffset(long endOffset) { + this.endOffset = endOffset; + } + + public List getBufferList() { + return bufferList; + } + + public void setBufferList(List bufferList) { + this.bufferList = bufferList; + } + + public List getDispatchRequests() { + return dispatchRequests; + } + + public void setDispatchRequests(List dispatchRequests) { + this.dispatchRequests = dispatchRequests; + } + + public void release() { + if (bufferList != null) { + for (SelectMappedBufferResult bufferResult : bufferList) { + bufferResult.release(); + } + bufferList.clear(); + bufferList = null; + } + if (dispatchRequests != null) { + dispatchRequests.clear(); + dispatchRequests = null; + } + + } +} diff --git a/tieredstore/src/main/java/org/apache/rocketmq/tieredstore/core/MessageStoreDispatcherImpl.java b/tieredstore/src/main/java/org/apache/rocketmq/tieredstore/core/MessageStoreDispatcherImpl.java index 9b1e53564d7..bcc4e225da2 100644 --- a/tieredstore/src/main/java/org/apache/rocketmq/tieredstore/core/MessageStoreDispatcherImpl.java +++ b/tieredstore/src/main/java/org/apache/rocketmq/tieredstore/core/MessageStoreDispatcherImpl.java @@ -16,15 +16,20 @@ */ package org.apache.rocketmq.tieredstore.core; +import com.google.common.annotations.VisibleForTesting; import io.opentelemetry.api.common.Attributes; import java.nio.ByteBuffer; import java.time.Duration; +import java.util.ArrayList; import java.util.Arrays; import java.util.HashMap; import java.util.HashSet; +import java.util.Iterator; +import java.util.List; import java.util.Map; import java.util.Set; import java.util.concurrent.CompletableFuture; +import java.util.concurrent.ConcurrentHashMap; import java.util.concurrent.Semaphore; import java.util.concurrent.TimeUnit; import org.apache.commons.lang3.StringUtils; @@ -42,6 +47,7 @@ import org.apache.rocketmq.tieredstore.TieredMessageStore; import org.apache.rocketmq.tieredstore.common.AppendResult; import org.apache.rocketmq.tieredstore.common.FileSegmentType; +import org.apache.rocketmq.tieredstore.common.GroupCommitContext; import org.apache.rocketmq.tieredstore.file.FlatFileInterface; import org.apache.rocketmq.tieredstore.file.FlatFileStore; import org.apache.rocketmq.tieredstore.index.IndexService; @@ -65,6 +71,7 @@ public class MessageStoreDispatcherImpl extends ServiceThread implements Message protected final MessageStoreFilter topicFilter; protected final Semaphore semaphore; protected final IndexService indexService; + protected final Map failedGroupCommitMap; public MessageStoreDispatcherImpl(TieredMessageStore messageStore) { this.messageStore = messageStore; @@ -77,6 +84,7 @@ public MessageStoreDispatcherImpl(TieredMessageStore messageStore) { this.flatFileStore = messageStore.getFlatFileStore(); this.storeExecutor = messageStore.getStoreExecutor(); this.indexService = messageStore.getIndexService(); + this.failedGroupCommitMap = new ConcurrentHashMap<>(); } @Override @@ -84,6 +92,11 @@ public String getServiceName() { return MessageStoreDispatcher.class.getSimpleName(); } + @VisibleForTesting + public Map getFailedGroupCommitMap() { + return failedGroupCommitMap; + } + public void dispatchWithSemaphore(FlatFileInterface flatFile) { try { if (stopped) { @@ -153,10 +166,22 @@ public CompletableFuture doScheduleDispatch(FlatFileInterface flatFile, // If the previous commit fails, attempt to trigger a commit directly. if (commitOffset < currentOffset) { - this.commitAsync(flatFile); + this.commitAsync(flatFile).whenComplete((result, throwable) -> { + if (throwable != null) { + log.error("MessageDispatcher#flatFile commitOffset less than currentOffset, commitAsync again failed. topic: {}, queueId: {} ", topic, queueId, throwable); + } + }); return CompletableFuture.completedFuture(false); } + if (failedGroupCommitMap.containsKey(flatFile)) { + GroupCommitContext failedCommit = failedGroupCommitMap.get(flatFile); + if (failedCommit.getEndOffset() <= commitOffset) { + failedGroupCommitMap.remove(flatFile); + constructIndexFile(flatFile.getTopicId(), failedCommit); + } + } + if (currentOffset < minOffsetInQueue) { log.warn("MessageDispatcher#dispatch, current offset is too small, topic={}, queueId={}, offset={}-{}, current={}", topic, queueId, minOffsetInQueue, maxOffsetInQueue, currentOffset); @@ -224,6 +249,8 @@ public CompletableFuture doScheduleDispatch(FlatFileInterface flatFile, } long offset = currentOffset; + List appendingBufferList = new ArrayList<>(); + List dispatchRequestList = new ArrayList<>(); for (; offset < targetOffset; offset++) { cqUnit = consumeQueue.get(offset); bufferSize += cqUnit.getSize(); @@ -231,6 +258,7 @@ public CompletableFuture doScheduleDispatch(FlatFileInterface flatFile, break; } message = defaultStore.selectOneMessageByOffset(cqUnit.getPos(), cqUnit.getSize()); + appendingBufferList.add(message); ByteBuffer byteBuffer = message.getByteBuffer(); AppendResult result = flatFile.appendCommitLog(message); @@ -251,13 +279,20 @@ public CompletableFuture doScheduleDispatch(FlatFileInterface flatFile, result = flatFile.appendConsumeQueue(dispatchRequest); if (!AppendResult.SUCCESS.equals(result)) { break; + } else { + dispatchRequestList.add(dispatchRequest); } } + GroupCommitContext groupCommitContext = new GroupCommitContext(); + groupCommitContext.setEndOffset(offset); + groupCommitContext.setBufferList(appendingBufferList); + groupCommitContext.setDispatchRequests(dispatchRequestList); + // If there are many messages waiting to be uploaded, call the upload logic immediately. boolean repeat = timeout || maxOffsetInQueue - offset > storeConfig.getTieredStoreGroupCommitCount(); - if (!flatFile.getDispatchRequestList().isEmpty()) { + if (!dispatchRequestList.isEmpty()) { Attributes attributes = TieredStoreMetricsManager.newAttributesBuilder() .put(TieredStoreMetricsConstant.LABEL_TOPIC, topic) .put(TieredStoreMetricsConstant.LABEL_QUEUE_ID, queueId) @@ -265,8 +300,19 @@ public CompletableFuture doScheduleDispatch(FlatFileInterface flatFile, .build(); TieredStoreMetricsManager.messagesDispatchTotal.add(offset - currentOffset, attributes); - this.commitAsync(flatFile).whenComplete((unused, throwable) -> { - if (repeat) { + this.commitAsync(flatFile).whenComplete((success, throwable) -> { + if (success) { + constructIndexFile(flatFile.getTopicId(), groupCommitContext); + } + else { + //next commit async,execute constructIndexFile. + GroupCommitContext oldCommit = failedGroupCommitMap.put(flatFile, groupCommitContext); + if (oldCommit != null) { + log.warn("MessageDispatcher#commitAsync failed,flatFile old failed commit context not release, topic={}, queueId={} ", topic, queueId); + oldCommit.release(); + } + } + if (success && repeat) { storeExecutor.commonExecutor.submit(() -> dispatchWithSemaphore(flatFile)); } } @@ -282,22 +328,28 @@ public CompletableFuture doScheduleDispatch(FlatFileInterface flatFile, return CompletableFuture.completedFuture(false); } - public CompletableFuture commitAsync(FlatFileInterface flatFile) { - return flatFile.commitAsync().thenAcceptAsync(success -> { - if (success) { - if (storeConfig.isMessageIndexEnable()) { - flatFile.getDispatchRequestList().forEach( - request -> constructIndexFile(flatFile.getTopicId(), request)); + public CompletableFuture commitAsync(FlatFileInterface flatFile) { + return flatFile.commitAsync(); + } + + public void constructIndexFile(long topicId, GroupCommitContext groupCommitContext) { + MessageStoreExecutor.getInstance().bufferCommitExecutor.submit(() -> { + if (storeConfig.isMessageIndexEnable()) { + try { + groupCommitContext.getDispatchRequests().forEach(request -> constructIndexFile0(topicId, request)); + } + catch (Throwable e) { + log.error("constructIndexFile error {}", topicId, e); } - flatFile.release(); } - }, storeExecutor.bufferCommitExecutor); + groupCommitContext.release(); + }); } /** * Building indexes with offsetId is no longer supported because offsetId has changed in tiered storage */ - public void constructIndexFile(long topicId, DispatchRequest request) { + public void constructIndexFile0(long topicId, DispatchRequest request) { Set keySet = new HashSet<>(); if (StringUtils.isNotBlank(request.getUniqKey())) { keySet.add(request.getUniqKey()); @@ -309,12 +361,27 @@ public void constructIndexFile(long topicId, DispatchRequest request) { request.getCommitLogOffset(), request.getMsgSize(), request.getStoreTimestamp()); } + public void releaseClosedPendingGroupCommit() { + Iterator> iterator = failedGroupCommitMap.entrySet().iterator(); + while (iterator.hasNext()) { + Map.Entry entry = iterator.next(); + if (entry.getKey().isClosed()) { + entry.getValue().release(); + iterator.remove(); + } + } + } + + @Override public void run() { log.info("{} service started", this.getServiceName()); while (!this.isStopped()) { try { flatFileStore.deepCopyFlatFileToList().forEach(this::dispatchWithSemaphore); + + releaseClosedPendingGroupCommit(); + this.waitForRunning(Duration.ofSeconds(20).toMillis()); } catch (Throwable t) { log.error("MessageStore dispatch error", t); diff --git a/tieredstore/src/main/java/org/apache/rocketmq/tieredstore/core/MessageStoreFetcherImpl.java b/tieredstore/src/main/java/org/apache/rocketmq/tieredstore/core/MessageStoreFetcherImpl.java index 7f79dbcd984..bc347bd5b47 100644 --- a/tieredstore/src/main/java/org/apache/rocketmq/tieredstore/core/MessageStoreFetcherImpl.java +++ b/tieredstore/src/main/java/org/apache/rocketmq/tieredstore/core/MessageStoreFetcherImpl.java @@ -148,6 +148,10 @@ protected GetMessageResultExt getMessageFromCache( if (result.getMessageCount() == maxCount) { break; } + long maxTransferBytes = messageStore.getMessageStoreConfig().getMaxTransferBytesOnMessageInMemory(); + if (result.getBufferTotalSize() >= maxTransferBytes) { + break; + } } result.setStatus(result.getMessageCount() > 0 ? GetMessageStatus.FOUND : GetMessageStatus.NO_MATCHED_MESSAGE); diff --git a/tieredstore/src/main/java/org/apache/rocketmq/tieredstore/file/FlatFileInterface.java b/tieredstore/src/main/java/org/apache/rocketmq/tieredstore/file/FlatFileInterface.java index 619470fbc27..01e7f25a467 100644 --- a/tieredstore/src/main/java/org/apache/rocketmq/tieredstore/file/FlatFileInterface.java +++ b/tieredstore/src/main/java/org/apache/rocketmq/tieredstore/file/FlatFileInterface.java @@ -17,7 +17,6 @@ package org.apache.rocketmq.tieredstore.file; import java.nio.ByteBuffer; -import java.util.List; import java.util.concurrent.CompletableFuture; import java.util.concurrent.locks.Lock; import org.apache.rocketmq.common.BoundaryType; @@ -58,8 +57,6 @@ public interface FlatFileInterface { */ AppendResult appendConsumeQueue(DispatchRequest request); - List getDispatchRequestList(); - void release(); long getMinStoreTimestamp(); @@ -143,6 +140,8 @@ public interface FlatFileInterface { */ CompletableFuture getQueueOffsetByTimeAsync(long timestamp, BoundaryType boundaryType); + boolean isClosed(); + /** * Shutdown process */ diff --git a/tieredstore/src/main/java/org/apache/rocketmq/tieredstore/file/FlatMessageFile.java b/tieredstore/src/main/java/org/apache/rocketmq/tieredstore/file/FlatMessageFile.java index d5675976cb1..4510a8a1271 100644 --- a/tieredstore/src/main/java/org/apache/rocketmq/tieredstore/file/FlatMessageFile.java +++ b/tieredstore/src/main/java/org/apache/rocketmq/tieredstore/file/FlatMessageFile.java @@ -17,12 +17,14 @@ package org.apache.rocketmq.tieredstore.file; import com.alibaba.fastjson.JSON; +import com.google.common.annotations.VisibleForTesting; import java.nio.ByteBuffer; import java.util.ArrayList; import java.util.List; import java.util.concurrent.CompletableFuture; import java.util.concurrent.ConcurrentHashMap; import java.util.concurrent.ConcurrentMap; +import java.util.concurrent.Semaphore; import java.util.concurrent.atomic.AtomicLong; import java.util.concurrent.locks.Lock; import java.util.concurrent.locks.ReentrantLock; @@ -51,14 +53,13 @@ public class FlatMessageFile implements FlatFileInterface { protected final String filePath; protected final ReentrantLock fileLock; + protected final Semaphore commitLock = new Semaphore(1); protected final MessageStoreConfig storeConfig; protected final MetadataStore metadataStore; protected final FlatCommitLogFile commitLog; protected final FlatConsumeQueueFile consumeQueue; protected final AtomicLong lastDestroyTime; - protected final List bufferResultList; - protected final List dispatchRequestList; protected final ConcurrentMap> inFlightRequestMap; public FlatMessageFile(FlatFileFactory fileFactory, String topic, int queueId) { @@ -76,8 +77,6 @@ public FlatMessageFile(FlatFileFactory fileFactory, String filePath) { this.commitLog = fileFactory.createFlatFileForCommitLog(filePath); this.consumeQueue = fileFactory.createFlatFileForConsumeQueue(filePath); this.lastDestroyTime = new AtomicLong(); - this.bufferResultList = new ArrayList<>(); - this.dispatchRequestList = new ArrayList<>(); this.inFlightRequestMap = new ConcurrentHashMap<>(); } @@ -127,6 +126,11 @@ public Lock getFileLock() { return this.fileLock; } + @VisibleForTesting + public Semaphore getCommitLock() { + return commitLock; + } + @Override public boolean rollingFile(long interval) { return this.commitLog.tryRollingFile(interval); @@ -156,7 +160,6 @@ public AppendResult appendCommitLog(SelectMappedBufferResult message) { if (closed) { return AppendResult.FILE_CLOSED; } - this.bufferResultList.add(message); return this.appendCommitLog(message.getByteBuffer()); } @@ -172,29 +175,14 @@ public AppendResult appendConsumeQueue(DispatchRequest request) { buffer.putLong(request.getTagsCode()); buffer.flip(); - this.dispatchRequestList.add(request); return consumeQueue.append(buffer, request.getStoreTimestamp()); } - @Override - public List getDispatchRequestList() { - return dispatchRequestList; - } + @Override public void release() { - for (SelectMappedBufferResult bufferResult : bufferResultList) { - bufferResult.release(); - } - - if (queueMetadata != null) { - log.trace("FlatMessageFile release, topic={}, queueId={}, bufferSize={}, requestListSize={}", - queueMetadata.getQueue().getTopic(), queueMetadata.getQueue().getQueueId(), - bufferResultList.size(), dispatchRequestList.size()); - } - bufferResultList.clear(); - dispatchRequestList.clear(); } @Override @@ -246,13 +234,18 @@ public long getConsumeQueueCommitOffset() { @Override public CompletableFuture commitAsync() { + // acquire lock + if (commitLock.drainPermits() <= 0) { + return CompletableFuture.completedFuture(false); + } + return this.commitLog.commitAsync() .thenCompose(result -> { if (result) { return consumeQueue.commitAsync(); } return CompletableFuture.completedFuture(false); - }); + }).whenComplete((result, throwable) -> commitLock.release()); } @Override @@ -363,6 +356,11 @@ public boolean equals(Object obj) { return StringUtils.equals(filePath, ((FlatMessageFile) obj).filePath); } + @Override + public boolean isClosed() { + return closed; + } + @Override public void shutdown() { closed = true; diff --git a/tieredstore/src/test/java/org/apache/rocketmq/tieredstore/common/GroupCommitContextTest.java b/tieredstore/src/test/java/org/apache/rocketmq/tieredstore/common/GroupCommitContextTest.java new file mode 100644 index 00000000000..e692360761d --- /dev/null +++ b/tieredstore/src/test/java/org/apache/rocketmq/tieredstore/common/GroupCommitContextTest.java @@ -0,0 +1,54 @@ +/* + * 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.rocketmq.tieredstore.common; + +import java.nio.ByteBuffer; +import java.util.ArrayList; +import java.util.List; +import org.apache.rocketmq.store.DispatchRequest; +import org.apache.rocketmq.store.SelectMappedBufferResult; +import org.junit.Assert; +import org.junit.Test; + +public class GroupCommitContextTest { + + @Test + public void groupCommitContextTest() { + GroupCommitContext releaseGroupCommitContext = new GroupCommitContext(); + releaseGroupCommitContext.release(); + + long endOffset = 1000; + List dispatchRequestList = new ArrayList<>(); + dispatchRequestList.add(new DispatchRequest(1000)); + List selectMappedBufferResultList = new ArrayList<>(); + selectMappedBufferResultList.add(new SelectMappedBufferResult(100, ByteBuffer.allocate(10), 1000, null)); + GroupCommitContext groupCommitContext = new GroupCommitContext(); + groupCommitContext.setEndOffset(endOffset); + groupCommitContext.setBufferList(selectMappedBufferResultList); + groupCommitContext.setDispatchRequests(dispatchRequestList); + + Assert.assertTrue(groupCommitContext.getEndOffset() == endOffset); + Assert.assertTrue(groupCommitContext.getBufferList().equals(selectMappedBufferResultList)); + Assert.assertTrue(groupCommitContext.getDispatchRequests().equals(dispatchRequestList)); + groupCommitContext.release(); + Assert.assertTrue(groupCommitContext.getDispatchRequests() == null); + Assert.assertTrue(groupCommitContext.getBufferList() == null); + Assert.assertTrue(dispatchRequestList.isEmpty()); + Assert.assertTrue(selectMappedBufferResultList.isEmpty()); + } + +} diff --git a/tieredstore/src/test/java/org/apache/rocketmq/tieredstore/core/MessageStoreDispatcherImplTest.java b/tieredstore/src/test/java/org/apache/rocketmq/tieredstore/core/MessageStoreDispatcherImplTest.java index 92e989e596f..7a43e1ede83 100644 --- a/tieredstore/src/test/java/org/apache/rocketmq/tieredstore/core/MessageStoreDispatcherImplTest.java +++ b/tieredstore/src/test/java/org/apache/rocketmq/tieredstore/core/MessageStoreDispatcherImplTest.java @@ -35,6 +35,7 @@ import org.apache.rocketmq.tieredstore.MessageStoreConfig; import org.apache.rocketmq.tieredstore.MessageStoreExecutor; import org.apache.rocketmq.tieredstore.TieredMessageStore; +import org.apache.rocketmq.tieredstore.common.GroupCommitContext; import org.apache.rocketmq.tieredstore.file.FlatFileFactory; import org.apache.rocketmq.tieredstore.file.FlatFileStore; import org.apache.rocketmq.tieredstore.file.FlatMessageFile; @@ -105,6 +106,7 @@ public void dispatchFromCommitLogTest() throws Exception { Mockito.when(messageStore.getStoreExecutor()).thenReturn(executor); Mockito.when(messageStore.getFlatFileStore()).thenReturn(fileStore); Mockito.when(messageStore.getIndexService()).thenReturn(indexService); + Mockito.when(messageStore.getMessageStoreConfig()).thenReturn(new org.apache.rocketmq.store.config.MessageStoreConfig()); // mock message ByteBuffer buffer = MessageFormatUtilTest.buildMockedMessageBuffer(); @@ -157,6 +159,130 @@ public void dispatchFromCommitLogTest() throws Exception { Assert.assertEquals(200L, flatFile.getConsumeQueueCommitOffset()); } + @Test + public void dispatchCommitFailedTest() throws Exception { + MessageStore defaultStore = Mockito.mock(MessageStore.class); + Mockito.when(defaultStore.getMinOffsetInQueue(anyString(), anyInt())).thenReturn(100L); + Mockito.when(defaultStore.getMaxOffsetInQueue(anyString(), anyInt())).thenReturn(200L); + + messageStore = Mockito.mock(TieredMessageStore.class); + IndexService indexService = + new IndexStoreService(new FlatFileFactory(metadataStore, storeConfig), storePath); + indexService.start(); + Mockito.when(messageStore.getDefaultStore()).thenReturn(defaultStore); + Mockito.when(messageStore.getStoreConfig()).thenReturn(storeConfig); + Mockito.when(messageStore.getStoreExecutor()).thenReturn(executor); + Mockito.when(messageStore.getFlatFileStore()).thenReturn(fileStore); + Mockito.when(messageStore.getIndexService()).thenReturn(indexService); + + // mock message + ByteBuffer buffer = MessageFormatUtilTest.buildMockedMessageBuffer(); + MessageExt messageExt = MessageDecoder.decode(buffer); + messageExt.setKeys("Key"); + MessageAccessor.putProperty( + messageExt, MessageConst.PROPERTY_UNIQ_CLIENT_MESSAGE_ID_KEYIDX, "uk"); + messageExt.setBody(new byte[10]); + messageExt.setStoreSize(0); + buffer = ByteBuffer.wrap(MessageDecoder.encode(messageExt, false)); + buffer.putInt(0, buffer.remaining()); + + DispatchRequest request = new DispatchRequest(mq.getTopic(), mq.getQueueId(), + MessageFormatUtil.getCommitLogOffset(buffer), buffer.remaining(), 0L, + MessageFormatUtil.getStoreTimeStamp(buffer), 0L, + "", "", 0, 0L, new HashMap<>()); + + // construct flat file + MessageStoreDispatcher dispatcher = new MessageStoreDispatcherImpl(messageStore); + dispatcher.dispatch(request); + FlatMessageFile flatFile = fileStore.getFlatFile(mq); + Assert.assertNotNull(flatFile); + + // init offset + dispatcher.doScheduleDispatch(flatFile, true).join(); + Assert.assertEquals(100L, flatFile.getConsumeQueueMinOffset()); + Assert.assertEquals(100L, flatFile.getConsumeQueueMaxOffset()); + Assert.assertEquals(100L, flatFile.getConsumeQueueCommitOffset()); + + ConsumeQueueInterface cq = Mockito.mock(ConsumeQueueInterface.class); + Mockito.when(defaultStore.getConsumeQueue(anyString(), anyInt())).thenReturn(cq); + Mockito.when(cq.get(anyLong())).thenReturn( + new CqUnit(100, 1000, buffer.remaining(), 0L)); + Mockito.when(defaultStore.selectOneMessageByOffset(anyLong(), anyInt())).thenReturn( + new SelectMappedBufferResult(0L, buffer.asReadOnlyBuffer(), buffer.remaining(), null)); + flatFile.getCommitLock().drainPermits(); + dispatcher.doScheduleDispatch(flatFile, true).join(); + GroupCommitContext groupCommitContext = ((MessageStoreDispatcherImpl)dispatcher).getFailedGroupCommitMap().get(flatFile); + Assert.assertTrue(groupCommitContext != null); + Assert.assertTrue(groupCommitContext.getEndOffset() == 200); + flatFile.getCommitLock().release(); + flatFile.commitAsync().join(); + dispatcher.doScheduleDispatch(flatFile, true).join(); + Assert.assertTrue(((MessageStoreDispatcherImpl)dispatcher).getFailedGroupCommitMap().get(flatFile) == null); + ((MessageStoreDispatcherImpl)dispatcher).flatFileStore.destroyFile(mq); + ((MessageStoreDispatcherImpl)dispatcher).releaseClosedPendingGroupCommit(); + + } + + @Test + public void dispatchFailedGroupCommitMapReleaseTest() throws Exception { + MessageStore defaultStore = Mockito.mock(MessageStore.class); + Mockito.when(defaultStore.getMinOffsetInQueue(anyString(), anyInt())).thenReturn(100L); + Mockito.when(defaultStore.getMaxOffsetInQueue(anyString(), anyInt())).thenReturn(200L); + + messageStore = Mockito.mock(TieredMessageStore.class); + IndexService indexService = + new IndexStoreService(new FlatFileFactory(metadataStore, storeConfig), storePath); + indexService.start(); + Mockito.when(messageStore.getDefaultStore()).thenReturn(defaultStore); + Mockito.when(messageStore.getStoreConfig()).thenReturn(storeConfig); + Mockito.when(messageStore.getStoreExecutor()).thenReturn(executor); + Mockito.when(messageStore.getFlatFileStore()).thenReturn(fileStore); + Mockito.when(messageStore.getIndexService()).thenReturn(indexService); + + // mock message + ByteBuffer buffer = MessageFormatUtilTest.buildMockedMessageBuffer(); + MessageExt messageExt = MessageDecoder.decode(buffer); + messageExt.setKeys("Key"); + MessageAccessor.putProperty( + messageExt, MessageConst.PROPERTY_UNIQ_CLIENT_MESSAGE_ID_KEYIDX, "uk"); + messageExt.setBody(new byte[10]); + messageExt.setStoreSize(0); + buffer = ByteBuffer.wrap(MessageDecoder.encode(messageExt, false)); + buffer.putInt(0, buffer.remaining()); + + DispatchRequest request = new DispatchRequest(mq.getTopic(), mq.getQueueId(), + MessageFormatUtil.getCommitLogOffset(buffer), buffer.remaining(), 0L, + MessageFormatUtil.getStoreTimeStamp(buffer), 0L, + "", "", 0, 0L, new HashMap<>()); + + // construct flat file + MessageStoreDispatcher dispatcher = new MessageStoreDispatcherImpl(messageStore); + dispatcher.dispatch(request); + FlatMessageFile flatFile = fileStore.getFlatFile(mq); + Assert.assertNotNull(flatFile); + + // init offset + dispatcher.doScheduleDispatch(flatFile, true).join(); + Assert.assertEquals(100L, flatFile.getConsumeQueueMinOffset()); + Assert.assertEquals(100L, flatFile.getConsumeQueueMaxOffset()); + Assert.assertEquals(100L, flatFile.getConsumeQueueCommitOffset()); + + ConsumeQueueInterface cq = Mockito.mock(ConsumeQueueInterface.class); + Mockito.when(defaultStore.getConsumeQueue(anyString(), anyInt())).thenReturn(cq); + Mockito.when(cq.get(anyLong())).thenReturn( + new CqUnit(100, 1000, buffer.remaining(), 0L)); + Mockito.when(defaultStore.selectOneMessageByOffset(anyLong(), anyInt())).thenReturn( + new SelectMappedBufferResult(0L, buffer.asReadOnlyBuffer(), buffer.remaining(), null)); + flatFile.getCommitLock().drainPermits(); + dispatcher.doScheduleDispatch(flatFile, true).join(); + GroupCommitContext groupCommitContext = ((MessageStoreDispatcherImpl)dispatcher).getFailedGroupCommitMap().get(flatFile); + Assert.assertTrue(groupCommitContext != null); + ((MessageStoreDispatcherImpl)dispatcher).flatFileStore.destroyFile(mq); + ((MessageStoreDispatcherImpl)dispatcher).releaseClosedPendingGroupCommit(); + Assert.assertTrue(((MessageStoreDispatcherImpl)dispatcher).getFailedGroupCommitMap().get(flatFile) == null); + + } + @Test public void dispatchServiceTest() { MessageStore defaultStore = Mockito.mock(MessageStore.class); diff --git a/tieredstore/src/test/java/org/apache/rocketmq/tieredstore/file/FlatMessageFileTest.java b/tieredstore/src/test/java/org/apache/rocketmq/tieredstore/file/FlatMessageFileTest.java index 8a417f54a74..8208d277415 100644 --- a/tieredstore/src/test/java/org/apache/rocketmq/tieredstore/file/FlatMessageFileTest.java +++ b/tieredstore/src/test/java/org/apache/rocketmq/tieredstore/file/FlatMessageFileTest.java @@ -216,4 +216,12 @@ public void testBinarySearchInQueueByTime() { flatFile.destroy(); } + + @Test + public void testCommitLock() { + String topic = "CommitLogTest"; + FlatMessageFile flatFile = new FlatMessageFile(flatFileFactory, topic, 0); + flatFile.getCommitLock().drainPermits(); + Assert.assertFalse(flatFile.commitAsync().join()); + } } diff --git a/tools/src/main/java/org/apache/rocketmq/tools/admin/DefaultMQAdminExt.java b/tools/src/main/java/org/apache/rocketmq/tools/admin/DefaultMQAdminExt.java index c5ecdefb529..f224f749cbc 100644 --- a/tools/src/main/java/org/apache/rocketmq/tools/admin/DefaultMQAdminExt.java +++ b/tools/src/main/java/org/apache/rocketmq/tools/admin/DefaultMQAdminExt.java @@ -65,6 +65,7 @@ import org.apache.rocketmq.remoting.protocol.body.TopicConfigSerializeWrapper; import org.apache.rocketmq.remoting.protocol.body.TopicList; import org.apache.rocketmq.remoting.protocol.body.UserInfo; +import org.apache.rocketmq.remoting.protocol.header.ExportRocksDBConfigToJsonRequestHeader; import org.apache.rocketmq.remoting.protocol.header.controller.ElectMasterResponseHeader; import org.apache.rocketmq.remoting.protocol.header.controller.GetMetaDataResponseHeader; import org.apache.rocketmq.remoting.protocol.heartbeat.SubscriptionData; @@ -778,6 +779,13 @@ public CheckRocksdbCqWriteResult checkRocksdbCqWriteProgress(String brokerAddr, return this.defaultMQAdminExtImpl.checkRocksdbCqWriteProgress(brokerAddr, topic, checkStoreTime); } + @Override + public void exportRocksDBConfigToJson(String brokerAddr, + List configType) + throws InterruptedException, RemotingTimeoutException, RemotingSendRequestException, RemotingConnectException, MQClientException { + this.defaultMQAdminExtImpl.exportRocksDBConfigToJson(brokerAddr, configType); + } + @Override public boolean resumeCheckHalfMessage(String topic, String msgId) @@ -1004,4 +1012,10 @@ public List listAcl(String brokerAddr, String subjectFilter, String resourceFilter) throws RemotingConnectException, RemotingSendRequestException, RemotingTimeoutException, MQBrokerException, InterruptedException { return defaultMQAdminExtImpl.listAcl(brokerAddr, subjectFilter, resourceFilter); } + + @Override + public void exportPopRecords(String brokerAddr, long timeout) throws RemotingConnectException, + RemotingSendRequestException, RemotingTimeoutException, MQBrokerException, InterruptedException { + defaultMQAdminExtImpl.exportPopRecords(brokerAddr, timeout); + } } diff --git a/tools/src/main/java/org/apache/rocketmq/tools/admin/DefaultMQAdminExtImpl.java b/tools/src/main/java/org/apache/rocketmq/tools/admin/DefaultMQAdminExtImpl.java index 17f14f23af8..5be99606dc8 100644 --- a/tools/src/main/java/org/apache/rocketmq/tools/admin/DefaultMQAdminExtImpl.java +++ b/tools/src/main/java/org/apache/rocketmq/tools/admin/DefaultMQAdminExtImpl.java @@ -103,6 +103,7 @@ import org.apache.rocketmq.remoting.protocol.body.TopicConfigSerializeWrapper; import org.apache.rocketmq.remoting.protocol.body.TopicList; import org.apache.rocketmq.remoting.protocol.body.UserInfo; +import org.apache.rocketmq.remoting.protocol.header.ExportRocksDBConfigToJsonRequestHeader; import org.apache.rocketmq.remoting.protocol.header.UpdateConsumerOffsetRequestHeader; import org.apache.rocketmq.remoting.protocol.header.UpdateGroupForbiddenRequestHeader; import org.apache.rocketmq.remoting.protocol.header.controller.ElectMasterResponseHeader; @@ -1824,6 +1825,13 @@ public CheckRocksdbCqWriteResult checkRocksdbCqWriteProgress(String brokerAddr, return this.mqClientInstance.getMQClientAPIImpl().checkRocksdbCqWriteProgress(brokerAddr, topic, checkStoreTime, timeoutMillis); } + @Override + public void exportRocksDBConfigToJson(String brokerAddr, + List configType) + throws InterruptedException, RemotingTimeoutException, RemotingSendRequestException, RemotingConnectException, MQClientException { + this.mqClientInstance.getMQClientAPIImpl().exportRocksDBConfigToJson(brokerAddr, configType, timeoutMillis); + } + @Override public boolean resumeCheckHalfMessage(final String topic, final String msgId) throws RemotingException, MQClientException, InterruptedException, MQBrokerException { @@ -2085,4 +2093,10 @@ public List listAcl(String brokerAddr, String subjectFilter, String resourceFilter) throws RemotingConnectException, RemotingSendRequestException, RemotingTimeoutException, MQBrokerException, InterruptedException { return this.mqClientInstance.getMQClientAPIImpl().listAcl(brokerAddr, subjectFilter, resourceFilter, timeoutMillis); } + + @Override + public void exportPopRecords(String brokerAddr, long timeout) throws RemotingConnectException, + RemotingSendRequestException, RemotingTimeoutException, MQBrokerException, InterruptedException { + this.mqClientInstance.getMQClientAPIImpl().exportPopRecord(brokerAddr, timeout); + } } diff --git a/tools/src/main/java/org/apache/rocketmq/tools/admin/MQAdminExt.java b/tools/src/main/java/org/apache/rocketmq/tools/admin/MQAdminExt.java index aea43376eac..2f01b6cba81 100644 --- a/tools/src/main/java/org/apache/rocketmq/tools/admin/MQAdminExt.java +++ b/tools/src/main/java/org/apache/rocketmq/tools/admin/MQAdminExt.java @@ -61,6 +61,7 @@ import org.apache.rocketmq.remoting.protocol.body.TopicConfigSerializeWrapper; import org.apache.rocketmq.remoting.protocol.body.TopicList; import org.apache.rocketmq.remoting.protocol.body.UserInfo; +import org.apache.rocketmq.remoting.protocol.header.ExportRocksDBConfigToJsonRequestHeader; import org.apache.rocketmq.remoting.protocol.header.controller.ElectMasterResponseHeader; import org.apache.rocketmq.remoting.protocol.header.controller.GetMetaDataResponseHeader; import org.apache.rocketmq.remoting.protocol.heartbeat.SubscriptionData; @@ -392,6 +393,10 @@ QueryConsumeQueueResponseBody queryConsumeQueue(final String brokerAddr, final long index, final int count, final String consumerGroup) throws InterruptedException, RemotingTimeoutException, RemotingSendRequestException, RemotingConnectException, MQClientException; + void exportRocksDBConfigToJson(String brokerAddr, + List configType) + throws InterruptedException, RemotingTimeoutException, RemotingSendRequestException, RemotingConnectException, MQClientException; + boolean resumeCheckHalfMessage(final String topic, final String msgId) throws RemotingException, MQClientException, InterruptedException, MQBrokerException; @@ -526,4 +531,7 @@ String setCommitLogReadAheadMode(final String brokerAddr, String mode) AclInfo getAcl(String brokerAddr, String subject) throws RemotingConnectException, RemotingSendRequestException, RemotingTimeoutException, MQBrokerException, InterruptedException; List listAcl(String brokerAddr, String subjectFilter, String resourceFilter) throws RemotingConnectException, RemotingSendRequestException, RemotingTimeoutException, MQBrokerException, InterruptedException; + + void exportPopRecords(String brokerAddr, long timeout) throws RemotingConnectException, + RemotingSendRequestException, RemotingTimeoutException, MQBrokerException, InterruptedException; } diff --git a/tools/src/main/java/org/apache/rocketmq/tools/command/MQAdminStartup.java b/tools/src/main/java/org/apache/rocketmq/tools/command/MQAdminStartup.java index 313a777ce4f..a16c058ec44 100644 --- a/tools/src/main/java/org/apache/rocketmq/tools/command/MQAdminStartup.java +++ b/tools/src/main/java/org/apache/rocketmq/tools/command/MQAdminStartup.java @@ -80,6 +80,7 @@ import org.apache.rocketmq.tools.command.export.ExportMetadataCommand; import org.apache.rocketmq.tools.command.export.ExportMetadataInRocksDBCommand; import org.apache.rocketmq.tools.command.export.ExportMetricsCommand; +import org.apache.rocketmq.tools.command.export.ExportPopRecordCommand; import org.apache.rocketmq.tools.command.ha.GetSyncStateSetSubCommand; import org.apache.rocketmq.tools.command.ha.HAStatusSubCommand; import org.apache.rocketmq.tools.command.message.CheckMsgSendRTCommand; @@ -273,6 +274,7 @@ public static void initCommand() { initCommand(new ExportConfigsCommand()); initCommand(new ExportMetricsCommand()); initCommand(new ExportMetadataInRocksDBCommand()); + initCommand(new ExportPopRecordCommand()); initCommand(new HAStatusSubCommand()); diff --git a/tools/src/main/java/org/apache/rocketmq/tools/command/export/ExportMetadataInRocksDBCommand.java b/tools/src/main/java/org/apache/rocketmq/tools/command/export/ExportMetadataInRocksDBCommand.java index d5726985e3c..438d17d6689 100644 --- a/tools/src/main/java/org/apache/rocketmq/tools/command/export/ExportMetadataInRocksDBCommand.java +++ b/tools/src/main/java/org/apache/rocketmq/tools/command/export/ExportMetadataInRocksDBCommand.java @@ -18,6 +18,10 @@ package org.apache.rocketmq.tools.command.export; import com.alibaba.fastjson.JSONObject; +import java.util.HashMap; +import java.util.Map; +import java.util.concurrent.atomic.AtomicLong; +import java.util.function.BiConsumer; import org.apache.commons.cli.CommandLine; import org.apache.commons.cli.Option; import org.apache.commons.cli.Options; @@ -30,11 +34,6 @@ import org.apache.rocketmq.tools.command.SubCommandException; import org.rocksdb.RocksIterator; -import java.util.HashMap; -import java.util.Map; -import java.util.concurrent.atomic.AtomicLong; -import java.util.function.BiConsumer; - public class ExportMetadataInRocksDBCommand implements SubCommand { private static final String TOPICS_JSON_CONFIG = "topics"; private static final String SUBSCRIPTION_GROUP_JSON_CONFIG = "subscriptionGroups"; @@ -46,7 +45,7 @@ public String commandName() { @Override public String commandDesc() { - return "export RocksDB kv config (topics/subscriptionGroups)"; + return "export RocksDB kv config (topics/subscriptionGroups). Recommend to use [mqadmin rocksDBConfigToJson]"; } @Override diff --git a/tools/src/main/java/org/apache/rocketmq/tools/command/export/ExportPopRecordCommand.java b/tools/src/main/java/org/apache/rocketmq/tools/command/export/ExportPopRecordCommand.java new file mode 100644 index 00000000000..f8b67c97af3 --- /dev/null +++ b/tools/src/main/java/org/apache/rocketmq/tools/command/export/ExportPopRecordCommand.java @@ -0,0 +1,110 @@ +/* + * 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.rocketmq.tools.command.export; + +import java.util.Set; +import java.util.concurrent.TimeUnit; +import org.apache.commons.cli.CommandLine; +import org.apache.commons.cli.Option; +import org.apache.commons.cli.Options; +import org.apache.rocketmq.remoting.RPCHook; +import org.apache.rocketmq.remoting.protocol.body.ClusterInfo; +import org.apache.rocketmq.remoting.protocol.route.BrokerData; +import org.apache.rocketmq.tools.admin.DefaultMQAdminExt; +import org.apache.rocketmq.tools.command.SubCommand; +import org.apache.rocketmq.tools.command.SubCommandException; + +public class ExportPopRecordCommand implements SubCommand { + + @Override + public String commandName() { + return "exportPopRecord"; + } + + @Override + public String commandDesc() { + return "Export pop consumer record"; + } + + @Override + public Options buildCommandlineOptions(Options options) { + Option opt = new Option( + "c", "clusterName", true, "choose one cluster to export"); + opt.setRequired(false); + options.addOption(opt); + + opt = new Option("b", "brokerAddr", true, "choose one broker to export"); + opt.setRequired(false); + options.addOption(opt); + + opt = new Option("d", "dryRun", true, "no actual changes will be made"); + opt.setRequired(false); + options.addOption(opt); + return options; + } + + @Override + public void execute(CommandLine commandLine, Options options, RPCHook rpcHook) throws SubCommandException { + + DefaultMQAdminExt adminExt = new DefaultMQAdminExt(rpcHook); + adminExt.setInstanceName(Long.toString(System.currentTimeMillis())); + + try { + adminExt.start(); + boolean dryRun = commandLine.hasOption('d') && + Boolean.FALSE.toString().equalsIgnoreCase(commandLine.getOptionValue('d')); + if (commandLine.hasOption('b')) { + String brokerAddr = commandLine.getOptionValue('b').trim(); + String brokerName = adminExt.getBrokerConfig(brokerAddr).getProperty("brokerName"); + export(adminExt, brokerAddr, brokerName, dryRun); + } else if (commandLine.hasOption('c')) { + String clusterName = commandLine.getOptionValue('c').trim(); + ClusterInfo clusterInfo = adminExt.examineBrokerClusterInfo(); + if (clusterInfo != null) { + Set brokerNameSet = clusterInfo.getClusterAddrTable().get(clusterName); + if (brokerNameSet != null) { + brokerNameSet.forEach(brokerName -> { + BrokerData brokerData = clusterInfo.getBrokerAddrTable().get(brokerName); + if (brokerData != null) { + brokerData.getBrokerAddrs().forEach( + (brokerId, brokerAddr) -> export(adminExt, brokerAddr, brokerName, dryRun)); + } + }); + } + } + } + } catch (Exception e) { + throw new SubCommandException(this.getClass().getSimpleName() + " command failed", e); + } finally { + adminExt.shutdown(); + } + } + + private void export(DefaultMQAdminExt adminExt, String brokerAddr, String brokerName, boolean dryRun) { + try { + if (!dryRun) { + adminExt.exportPopRecords(brokerAddr, TimeUnit.SECONDS.toMillis(30)); + } + System.out.printf("Export broker records, " + + "brokerName=%s, brokerAddr=%s, dryRun=%s%n", brokerName, brokerAddr, dryRun); + } catch (Exception e) { + System.out.printf("Export broker records error, " + + "brokerName=%s, brokerAddr=%s, dryRun=%s%n%s", brokerName, brokerAddr, dryRun, e); + } + } +} + diff --git a/tools/src/main/java/org/apache/rocketmq/tools/command/metadata/RocksDBConfigToJsonCommand.java b/tools/src/main/java/org/apache/rocketmq/tools/command/metadata/RocksDBConfigToJsonCommand.java index f2803b0cbb3..48bc163678b 100644 --- a/tools/src/main/java/org/apache/rocketmq/tools/command/metadata/RocksDBConfigToJsonCommand.java +++ b/tools/src/main/java/org/apache/rocketmq/tools/command/metadata/RocksDBConfigToJsonCommand.java @@ -18,27 +18,38 @@ package org.apache.rocketmq.tools.command.metadata; import com.alibaba.fastjson.JSONObject; +import java.io.File; +import java.io.IOException; +import java.nio.file.Paths; +import java.util.ArrayList; +import java.util.Arrays; +import java.util.HashMap; +import java.util.List; +import java.util.Map; +import java.util.Objects; +import java.util.Set; +import java.util.concurrent.CompletableFuture; +import java.util.concurrent.CompletionException; +import java.util.concurrent.ConcurrentHashMap; +import java.util.concurrent.ConcurrentMap; +import java.util.concurrent.atomic.AtomicLong; import org.apache.commons.cli.CommandLine; import org.apache.commons.cli.Option; import org.apache.commons.cli.Options; import org.apache.commons.lang3.StringUtils; +import org.apache.rocketmq.common.MixAll; import org.apache.rocketmq.common.config.ConfigRocksDBStorage; import org.apache.rocketmq.common.utils.DataConverter; import org.apache.rocketmq.remoting.RPCHook; +import org.apache.rocketmq.remoting.protocol.body.ClusterInfo; +import org.apache.rocketmq.remoting.protocol.header.ExportRocksDBConfigToJsonRequestHeader; +import org.apache.rocketmq.remoting.protocol.route.BrokerData; +import org.apache.rocketmq.tools.admin.DefaultMQAdminExt; import org.apache.rocketmq.tools.command.SubCommand; import org.apache.rocketmq.tools.command.SubCommandException; import org.rocksdb.RocksIterator; -import java.io.File; -import java.util.HashMap; -import java.util.Map; -import java.util.concurrent.ConcurrentHashMap; -import java.util.concurrent.ConcurrentMap; - public class RocksDBConfigToJsonCommand implements SubCommand { - private static final String TOPICS_JSON_CONFIG = "topics"; - private static final String SUBSCRIPTION_GROUP_JSON_CONFIG = "subscriptionGroups"; - private static final String CONSUMER_OFFSETS_JSON_CONFIG = "consumerOffsets"; @Override public String commandName() { @@ -47,41 +58,140 @@ public String commandName() { @Override public String commandDesc() { - return "Convert RocksDB kv config (topics/subscriptionGroups/consumerOffsets) to json"; + return "Convert RocksDB kv config (topics/subscriptionGroups/consumerOffsets) to json. " + + "[rpc mode] Use [-n, -c, -b, -t] to send Request to broker ( version >= 5.3.2 ) or [local mode] use [-p, -t, -j, -e] to load RocksDB. " + + "If -e is provided, tools will export json file instead of std print"; } @Override public Options buildCommandlineOptions(Options options) { + Option configTypeOption = new Option("t", "configType", true, "Name of kv config, e.g. " + + "topics/subscriptionGroups/consumerOffsets. Required in local mode and default all in rpc mode."); + options.addOption(configTypeOption); + + // [local mode] options Option pathOption = new Option("p", "configPath", true, - "Absolute path to the metadata config directory"); - pathOption.setRequired(true); + "[local mode] Absolute path to the metadata config directory"); options.addOption(pathOption); - Option configTypeOption = new Option("t", "configType", true, "Name of kv config, e.g. " + - "topics/subscriptionGroups/consumerOffsets"); - configTypeOption.setRequired(true); - options.addOption(configTypeOption); + Option exportPathOption = new Option("e", "exportFile", true, + "[local mode] Absolute file path for exporting, auto backup existing file, not directory. If exportFile is provided, will export Json file and ignore [-j]."); + options.addOption(exportPathOption); + + Option jsonEnableOption = new Option("j", "jsonEnable", true, + "[local mode] Json format enable, Default: true. If exportFile is provided, will export Json file and ignore [-j]."); + options.addOption(jsonEnableOption); + + // [rpc mode] options + Option nameserverOption = new Option("n", "nameserverAddr", true, + "[rpc mode] nameserverAddr. If nameserverAddr and clusterName are provided, will ignore [-p, -e, -j, -b] args"); + options.addOption(nameserverOption); + + Option clusterOption = new Option("c", "cluster", true, + "[rpc mode] Cluster name. If nameserverAddr and clusterName are provided, will ignore [-p, -e, -j, -b] args"); + options.addOption(clusterOption); + + Option brokerAddrOption = new Option("b", "brokerAddr", true, + "[rpc mode] Broker address. If brokerAddr is provided, will ignore [-p, -e, -j] args"); + options.addOption(brokerAddrOption); return options; } @Override public void execute(CommandLine commandLine, Options options, RPCHook rpcHook) throws SubCommandException { + List typeList = getConfigTypeList(commandLine); + + if (commandLine.hasOption("nameserverAddr")) { + // [rpc mode] call all brokers in cluster to export to json file + System.out.print("Use [rpc mode] call all brokers in cluster to export to json file \n"); + checkRequiredArgsProvided(commandLine, "rpc mode", "cluster"); + handleRpcMode(commandLine, rpcHook, typeList); + } else if (commandLine.hasOption("brokerAddr")) { + // [rpc mode] call broker to export to json file + System.out.print("Use [rpc mode] call broker to export to json file \n"); + handleRpcMode(commandLine, rpcHook, typeList); + } else if (commandLine.hasOption("configPath")) { + // [local mode] load rocksdb to print or export file + System.out.print("Use [local mode] load rocksdb to print or export file \n"); + checkRequiredArgsProvided(commandLine, "local mode", "configType"); + handleLocalMode(commandLine); + } else { + System.out.print(commandDesc() + "\n"); + } + } + + private void handleLocalMode(CommandLine commandLine) { + ExportRocksDBConfigToJsonRequestHeader.ConfigType type = Objects.requireNonNull(getConfigTypeList(commandLine)).get(0); String path = commandLine.getOptionValue("configPath").trim(); if (StringUtils.isEmpty(path) || !new File(path).exists()) { System.out.print("Rocksdb path is invalid.\n"); return; } + path = Paths.get(path, type.toString()).toString(); + String exportFile = commandLine.hasOption("exportFile") ? commandLine.getOptionValue("exportFile").trim() : null; + Map configMap = getConfigMapFromRocksDB(path, type); + if (configMap != null) { + if (exportFile == null) { + if (commandLine.hasOption("jsonEnable") && "false".equalsIgnoreCase(commandLine.getOptionValue("jsonEnable").trim())) { + printConfigMapJsonDisable(configMap); + } else { + System.out.print(JSONObject.toJSONString(configMap, true) + "\n"); + } + } else { + String jsonString = JSONObject.toJSONString(configMap, true); + try { + MixAll.string2File(jsonString, exportFile); + } catch (IOException e) { + System.out.print("persist file " + exportFile + " exception" + e); + } + } + } + } - String configType = commandLine.getOptionValue("configType").trim(); - if (!path.endsWith("/")) { - path += "/"; + private void checkRequiredArgsProvided(CommandLine commandLine, String mode, + String... args) throws SubCommandException { + for (String arg : args) { + if (!commandLine.hasOption(arg)) { + System.out.printf("%s Invalid args, please input %s\n", mode, String.join(",", args)); + throw new SubCommandException("Invalid args"); + } } - path += configType; - if (CONSUMER_OFFSETS_JSON_CONFIG.equalsIgnoreCase(configType)) { - printConsumerOffsets(path); - return; + } + + private List getConfigTypeList(CommandLine commandLine) { + List typeList = new ArrayList<>(); + if (commandLine.hasOption("configType")) { + String configType = commandLine.getOptionValue("configType").trim(); + try { + typeList.addAll(ExportRocksDBConfigToJsonRequestHeader.ConfigType.fromString(configType)); + } catch (IllegalArgumentException e) { + System.out.print("Invalid configType: " + configType + " please input topics/subscriptionGroups/consumerOffsets \n"); + return null; + } + } else { + typeList.addAll(Arrays.asList(ExportRocksDBConfigToJsonRequestHeader.ConfigType.values())); } + return typeList; + } + + private static void printConfigMapJsonDisable(Map configMap) { + AtomicLong count = new AtomicLong(0); + for (Map.Entry entry : configMap.entrySet()) { + String configKey = entry.getKey(); + System.out.printf("type: %s", configKey); + JSONObject jsonObject = entry.getValue(); + jsonObject.forEach((k, v) -> System.out.printf("%d, Key: %s, Value: %s%n", count.incrementAndGet(), k, v)); + } + } + + private static Map getConfigMapFromRocksDB(String path, + ExportRocksDBConfigToJsonRequestHeader.ConfigType configType) { + + if (ExportRocksDBConfigToJsonRequestHeader.ConfigType.CONSUMER_OFFSETS.equals(configType)) { + return loadConsumerOffsets(path); + } + ConfigRocksDBStorage configRocksDBStorage = new ConfigRocksDBStorage(path, true); configRocksDBStorage.start(); RocksIterator iterator = configRocksDBStorage.iterator(); @@ -101,24 +211,79 @@ public void execute(CommandLine commandLine, Options options, RPCHook rpcHook) t byte[] kvDataVersion = configRocksDBStorage.getKvDataVersion(); if (kvDataVersion != null) { configMap.put("dataVersion", - JSONObject.parseObject(new String(kvDataVersion, DataConverter.CHARSET_UTF8))); + JSONObject.parseObject(new String(kvDataVersion, DataConverter.CHARSET_UTF8))); } - if (TOPICS_JSON_CONFIG.equalsIgnoreCase(configType)) { + if (ExportRocksDBConfigToJsonRequestHeader.ConfigType.TOPICS.equals(configType)) { configMap.put("topicConfigTable", configTable); } - if (SUBSCRIPTION_GROUP_JSON_CONFIG.equalsIgnoreCase(configType)) { + if (ExportRocksDBConfigToJsonRequestHeader.ConfigType.SUBSCRIPTION_GROUPS.equals(configType)) { configMap.put("subscriptionGroupTable", configTable); } - System.out.print(JSONObject.toJSONString(configMap, true) + "\n"); + return configMap; } catch (Exception e) { System.out.print("Error occurred while converting RocksDB kv config to json, " + "configType=" + configType + ", " + e.getMessage() + "\n"); } finally { configRocksDBStorage.shutdown(); } + return null; + } + + private void handleRpcMode(CommandLine commandLine, RPCHook rpcHook, + List type) { + String nameserverAddr = commandLine.hasOption('n') ? commandLine.getOptionValue("nameserverAddr").trim() : null; + String inputBrokerAddr = commandLine.hasOption('b') ? commandLine.getOptionValue('b').trim() : null; + String clusterName = commandLine.hasOption('c') ? commandLine.getOptionValue('c').trim() : null; + + DefaultMQAdminExt defaultMQAdminExt = new DefaultMQAdminExt(rpcHook, 30 * 1000); + defaultMQAdminExt.setInstanceName(Long.toString(System.currentTimeMillis())); + defaultMQAdminExt.setNamesrvAddr(nameserverAddr); + + List> futureList = new ArrayList<>(); + + try { + defaultMQAdminExt.start(); + if (clusterName != null) { + ClusterInfo clusterInfo = defaultMQAdminExt.examineBrokerClusterInfo(); + Map> clusterAddrTable = clusterInfo.getClusterAddrTable(); + Map brokerAddrTable = clusterInfo.getBrokerAddrTable(); + if (clusterAddrTable.get(clusterName) == null) { + System.out.print("clusterAddrTable is empty"); + return; + } + for (Map.Entry entry : brokerAddrTable.entrySet()) { + String brokerName = entry.getKey(); + BrokerData brokerData = entry.getValue(); + String brokerAddr = brokerData.getBrokerAddrs().get(0L); + futureList.add(sendRequest(type, defaultMQAdminExt, brokerAddr, brokerName)); + } + } else if (inputBrokerAddr != null) { + futureList.add(sendRequest(type, defaultMQAdminExt, inputBrokerAddr, null)); + } + CompletableFuture.allOf(futureList.toArray(new CompletableFuture[0])).whenComplete( + (v, t) -> System.out.print("broker export done.") + ).join(); + } catch (Exception e) { + throw new RuntimeException(this.getClass().getSimpleName() + " command failed", e); + } finally { + defaultMQAdminExt.shutdown(); + } + } + + private CompletableFuture sendRequest(List type, + DefaultMQAdminExt defaultMQAdminExt, String brokerAddr, String brokerName) { + return CompletableFuture.supplyAsync(() -> { + try { + defaultMQAdminExt.exportRocksDBConfigToJson(brokerAddr, type); + } catch (Throwable t) { + System.out.print((brokerName != null) ? brokerName : brokerAddr + " export error"); + throw new CompletionException(this.getClass().getSimpleName() + " command failed", t); + } + return null; + }); } - private void printConsumerOffsets(String path) { + private static Map loadConsumerOffsets(String path) { ConfigRocksDBStorage configRocksDBStorage = new ConfigRocksDBStorage(path, true); configRocksDBStorage.start(); RocksIterator iterator = configRocksDBStorage.iterator(); @@ -136,12 +301,13 @@ private void printConsumerOffsets(String path) { iterator.next(); } configMap.put("offsetTable", configTable); - System.out.print(JSONObject.toJSONString(configMap, true) + "\n"); + return configMap; } catch (Exception e) { System.out.print("Error occurred while converting RocksDB kv config to json, " + "configType=consumerOffsets, " + e.getMessage() + "\n"); } finally { configRocksDBStorage.shutdown(); } + return null; } static class RocksDBOffsetSerializeWrapper {