Skip to content
体验新版
项目
组织
正在加载...
登录
切换导航
打开侧边栏
Apache RocketMQ
Rocketmq
提交
ced6b023
R
Rocketmq
项目概览
Apache RocketMQ
/
Rocketmq
上一次同步 大约 3 年
通知
268
Star
16139
Fork
68
代码
文件
提交
分支
Tags
贡献者
分支图
Diff
Issue
0
列表
看板
标记
里程碑
合并请求
0
DevOps
流水线
流水线任务
计划
Wiki
0
Wiki
分析
仓库
DevOps
项目成员
Pages
R
Rocketmq
项目概览
项目概览
详情
发布
仓库
仓库
文件
提交
分支
标签
贡献者
分支图
比较
Issue
0
Issue
0
列表
看板
标记
里程碑
合并请求
0
合并请求
0
Pages
DevOps
DevOps
流水线
流水线任务
计划
分析
分析
仓库分析
DevOps
Wiki
0
Wiki
成员
成员
收起侧边栏
关闭侧边栏
动态
分支图
创建新Issue
流水线任务
提交
Issue看板
提交
ced6b023
编写于
3月 09, 2021
作者:
A
ayanamist
浏览文件
操作
浏览文件
下载
电子邮件补丁
差异文件
[RIP-19] Pop Consuming (broker)
上级
ea36854b
变更
26
隐藏空白更改
内联
并排
Showing
26 changed file
with
4616 addition
and
31 deletion
+4616
-31
broker/pom.xml
broker/pom.xml
+4
-0
broker/src/main/java/org/apache/rocketmq/broker/BrokerController.java
...ain/java/org/apache/rocketmq/broker/BrokerController.java
+134
-20
broker/src/main/java/org/apache/rocketmq/broker/BrokerPathConfigHelper.java
...va/org/apache/rocketmq/broker/BrokerPathConfigHelper.java
+8
-0
broker/src/main/java/org/apache/rocketmq/broker/loadbalance/AssignmentManager.java
...apache/rocketmq/broker/loadbalance/AssignmentManager.java
+149
-0
broker/src/main/java/org/apache/rocketmq/broker/loadbalance/MessageRequestModeManager.java
...ocketmq/broker/loadbalance/MessageRequestModeManager.java
+99
-0
broker/src/main/java/org/apache/rocketmq/broker/longpolling/NotifyMessageArrivingListener.java
...tmq/broker/longpolling/NotifyMessageArrivingListener.java
+6
-3
broker/src/main/java/org/apache/rocketmq/broker/longpolling/PopRequest.java
...va/org/apache/rocketmq/broker/longpolling/PopRequest.java
+87
-0
broker/src/main/java/org/apache/rocketmq/broker/offset/ConsumerOrderInfoManager.java
...ache/rocketmq/broker/offset/ConsumerOrderInfoManager.java
+426
-0
broker/src/main/java/org/apache/rocketmq/broker/out/BrokerOuterAPI.java
...n/java/org/apache/rocketmq/broker/out/BrokerOuterAPI.java
+40
-2
broker/src/main/java/org/apache/rocketmq/broker/processor/AckMessageProcessor.java
...apache/rocketmq/broker/processor/AckMessageProcessor.java
+188
-0
broker/src/main/java/org/apache/rocketmq/broker/processor/ChangeInvisibleTimeProcessor.java
...cketmq/broker/processor/ChangeInvisibleTimeProcessor.java
+195
-0
broker/src/main/java/org/apache/rocketmq/broker/processor/PopBufferMergeService.java
...ache/rocketmq/broker/processor/PopBufferMergeService.java
+731
-0
broker/src/main/java/org/apache/rocketmq/broker/processor/PopMessageProcessor.java
...apache/rocketmq/broker/processor/PopMessageProcessor.java
+967
-0
broker/src/main/java/org/apache/rocketmq/broker/processor/PopReviveService.java
...rg/apache/rocketmq/broker/processor/PopReviveService.java
+461
-0
broker/src/main/java/org/apache/rocketmq/broker/processor/QueryAssignmentProcessor.java
...e/rocketmq/broker/processor/QueryAssignmentProcessor.java
+307
-0
broker/src/main/java/org/apache/rocketmq/broker/util/MsgUtil.java
...rc/main/java/org/apache/rocketmq/broker/util/MsgUtil.java
+34
-0
broker/src/test/java/org/apache/rocketmq/broker/processor/AckMessageProcessorTest.java
...he/rocketmq/broker/processor/AckMessageProcessorTest.java
+132
-0
broker/src/test/java/org/apache/rocketmq/broker/processor/ChangeInvisibleTimeProcessorTest.java
...mq/broker/processor/ChangeInvisibleTimeProcessorTest.java
+133
-0
broker/src/test/java/org/apache/rocketmq/broker/processor/PopBufferMergeServiceTest.java
.../rocketmq/broker/processor/PopBufferMergeServiceTest.java
+108
-0
broker/src/test/java/org/apache/rocketmq/broker/processor/PopMessageProcessorTest.java
...he/rocketmq/broker/processor/PopMessageProcessorTest.java
+191
-0
broker/src/test/java/org/apache/rocketmq/broker/processor/QueryAssignmentProcessorTest.java
...cketmq/broker/processor/QueryAssignmentProcessorTest.java
+153
-0
client/src/main/java/org/apache/rocketmq/client/consumer/rebalance/AllocateMessageQueueAveragely.java
...ent/consumer/rebalance/AllocateMessageQueueAveragely.java
+10
-2
client/src/main/java/org/apache/rocketmq/client/consumer/rebalance/AllocateMessageQueueAveragelyByCircle.java
...umer/rebalance/AllocateMessageQueueAveragelyByCircle.java
+10
-2
distribution/conf/logback_broker.xml
distribution/conf/logback_broker.xml
+29
-0
pom.xml
pom.xml
+6
-1
remoting/src/main/java/org/apache/rocketmq/remoting/common/RemotingHelper.java
...a/org/apache/rocketmq/remoting/common/RemotingHelper.java
+8
-1
未找到文件。
broker/pom.xml
浏览文件 @
ced6b023
...
...
@@ -66,6 +66,10 @@
<groupId>
org.slf4j
</groupId>
<artifactId>
slf4j-api
</artifactId>
</dependency>
<dependency>
<groupId>
com.googlecode.concurrentlinkedhashmap
</groupId>
<artifactId>
concurrentlinkedhashmap-lru
</artifactId>
</dependency>
</dependencies>
<build>
...
...
broker/src/main/java/org/apache/rocketmq/broker/BrokerController.java
浏览文件 @
ced6b023
...
...
@@ -47,19 +47,25 @@ import org.apache.rocketmq.broker.filter.ConsumerFilterManager;
import
org.apache.rocketmq.broker.filtersrv.FilterServerManager
;
import
org.apache.rocketmq.broker.latency.BrokerFastFailure
;
import
org.apache.rocketmq.broker.latency.BrokerFixedThreadPoolExecutor
;
import
org.apache.rocketmq.broker.loadbalance.AssignmentManager
;
import
org.apache.rocketmq.broker.longpolling.NotifyMessageArrivingListener
;
import
org.apache.rocketmq.broker.longpolling.PullRequestHoldService
;
import
org.apache.rocketmq.broker.mqtrace.ConsumeMessageHook
;
import
org.apache.rocketmq.broker.mqtrace.SendMessageHook
;
import
org.apache.rocketmq.broker.offset.ConsumerOffsetManager
;
import
org.apache.rocketmq.broker.offset.ConsumerOrderInfoManager
;
import
org.apache.rocketmq.broker.out.BrokerOuterAPI
;
import
org.apache.rocketmq.broker.plugin.MessageStoreFactory
;
import
org.apache.rocketmq.broker.plugin.MessageStorePluginContext
;
import
org.apache.rocketmq.broker.processor.AckMessageProcessor
;
import
org.apache.rocketmq.broker.processor.AdminBrokerProcessor
;
import
org.apache.rocketmq.broker.processor.ChangeInvisibleTimeProcessor
;
import
org.apache.rocketmq.broker.processor.ClientManageProcessor
;
import
org.apache.rocketmq.broker.processor.ConsumerManageProcessor
;
import
org.apache.rocketmq.broker.processor.EndTransactionProcessor
;
import
org.apache.rocketmq.broker.processor.PopMessageProcessor
;
import
org.apache.rocketmq.broker.processor.PullMessageProcessor
;
import
org.apache.rocketmq.broker.processor.QueryAssignmentProcessor
;
import
org.apache.rocketmq.broker.processor.QueryMessageProcessor
;
import
org.apache.rocketmq.broker.processor.ReplyMessageProcessor
;
import
org.apache.rocketmq.broker.processor.SendMessageProcessor
;
...
...
@@ -118,9 +124,18 @@ public class BrokerController {
private
final
ConsumerOffsetManager
consumerOffsetManager
;
private
final
ConsumerManager
consumerManager
;
private
final
ConsumerFilterManager
consumerFilterManager
;
private
final
ConsumerOrderInfoManager
consumerOrderInfoManager
;
private
final
ProducerManager
producerManager
;
private
final
AssignmentManager
assignmentManager
;
private
final
ClientHousekeepingService
clientHousekeepingService
;
private
final
PullMessageProcessor
pullMessageProcessor
;
private
final
PopMessageProcessor
popMessageProcessor
;
private
final
AckMessageProcessor
ackMessageProcessor
;
private
final
ChangeInvisibleTimeProcessor
changeInvisibleTimeProcessor
;
private
final
QueryAssignmentProcessor
queryAssignmentProcessor
;
private
final
ClientManageProcessor
clientManageProcessor
;
private
final
SendMessageProcessor
sendMessageProcessor
;
private
final
PullRequestHoldService
pullRequestHoldService
;
private
final
MessageArrivingListener
messageArrivingListener
;
private
final
Broker2Client
broker2Client
;
...
...
@@ -132,6 +147,7 @@ public class BrokerController {
"BrokerControllerScheduledThread"
));
private
final
SlaveSynchronize
slaveSynchronize
;
private
final
BlockingQueue
<
Runnable
>
sendThreadPoolQueue
;
private
final
BlockingQueue
<
Runnable
>
ackThreadPoolQueue
;
private
final
BlockingQueue
<
Runnable
>
pullThreadPoolQueue
;
private
final
BlockingQueue
<
Runnable
>
replyThreadPoolQueue
;
private
final
BlockingQueue
<
Runnable
>
queryThreadPoolQueue
;
...
...
@@ -149,12 +165,14 @@ public class BrokerController {
private
TopicConfigManager
topicConfigManager
;
private
ExecutorService
sendMessageExecutor
;
private
ExecutorService
pullMessageExecutor
;
private
ExecutorService
ackMessageExecutor
;
private
ExecutorService
replyMessageExecutor
;
private
ExecutorService
queryMessageExecutor
;
private
ExecutorService
adminBrokerExecutor
;
private
ExecutorService
clientManageExecutor
;
private
ExecutorService
heartbeatExecutor
;
private
ExecutorService
consumerManageExecutor
;
private
ExecutorService
loadBalanceExecutor
;
private
ExecutorService
endTransactionExecutor
;
private
boolean
updateMasterHAServerAddrPeriodically
=
false
;
private
BrokerStats
brokerStats
;
...
...
@@ -167,6 +185,7 @@ public class BrokerController {
private
AbstractTransactionalMessageCheckListener
transactionalMessageCheckListener
;
private
Future
<?>
slaveSyncFuture
;
private
Map
<
Class
,
AccessValidator
>
accessValidatorMap
=
new
HashMap
<
Class
,
AccessValidator
>();
private
long
shouldStartTime
;
public
BrokerController
(
final
BrokerConfig
brokerConfig
,
...
...
@@ -182,10 +201,16 @@ public class BrokerController {
this
.
topicConfigManager
=
new
TopicConfigManager
(
this
);
this
.
pullMessageProcessor
=
new
PullMessageProcessor
(
this
);
this
.
pullRequestHoldService
=
new
PullRequestHoldService
(
this
);
this
.
messageArrivingListener
=
new
NotifyMessageArrivingListener
(
this
.
pullRequestHoldService
);
this
.
popMessageProcessor
=
new
PopMessageProcessor
(
this
);
this
.
ackMessageProcessor
=
new
AckMessageProcessor
(
this
);
this
.
changeInvisibleTimeProcessor
=
new
ChangeInvisibleTimeProcessor
(
this
);
this
.
sendMessageProcessor
=
new
SendMessageProcessor
(
this
);
this
.
messageArrivingListener
=
new
NotifyMessageArrivingListener
(
this
.
pullRequestHoldService
,
this
.
popMessageProcessor
);
this
.
consumerIdsChangeListener
=
new
DefaultConsumerIdsChangeListener
(
this
);
this
.
consumerManager
=
new
ConsumerManager
(
this
.
consumerIdsChangeListener
);
this
.
consumerFilterManager
=
new
ConsumerFilterManager
(
this
);
this
.
consumerOrderInfoManager
=
new
ConsumerOrderInfoManager
(
this
);
this
.
producerManager
=
new
ProducerManager
();
this
.
clientHousekeepingService
=
new
ClientHousekeepingService
(
this
);
this
.
broker2Client
=
new
Broker2Client
(
this
);
...
...
@@ -193,10 +218,14 @@ public class BrokerController {
this
.
brokerOuterAPI
=
new
BrokerOuterAPI
(
nettyClientConfig
);
this
.
filterServerManager
=
new
FilterServerManager
(
this
);
this
.
assignmentManager
=
new
AssignmentManager
(
this
);
this
.
queryAssignmentProcessor
=
new
QueryAssignmentProcessor
(
this
);
this
.
clientManageProcessor
=
new
ClientManageProcessor
(
this
);
this
.
slaveSynchronize
=
new
SlaveSynchronize
(
this
);
this
.
sendThreadPoolQueue
=
new
LinkedBlockingQueue
<
Runnable
>(
this
.
brokerConfig
.
getSendThreadPoolQueueCapacity
());
this
.
pullThreadPoolQueue
=
new
LinkedBlockingQueue
<
Runnable
>(
this
.
brokerConfig
.
getPullThreadPoolQueueCapacity
());
this
.
ackThreadPoolQueue
=
new
LinkedBlockingQueue
<
Runnable
>(
this
.
brokerConfig
.
getAckThreadPoolQueueCapacity
());
this
.
replyThreadPoolQueue
=
new
LinkedBlockingQueue
<
Runnable
>(
this
.
brokerConfig
.
getReplyThreadPoolQueueCapacity
());
this
.
queryThreadPoolQueue
=
new
LinkedBlockingQueue
<
Runnable
>(
this
.
brokerConfig
.
getQueryThreadPoolQueueCapacity
());
this
.
clientManagerThreadPoolQueue
=
new
LinkedBlockingQueue
<
Runnable
>(
this
.
brokerConfig
.
getClientManagerThreadPoolQueueCapacity
());
...
...
@@ -215,6 +244,14 @@ public class BrokerController {
);
}
public
ConsumerIdsChangeListener
getConsumerIdsChangeListener
()
{
return
consumerIdsChangeListener
;
}
public
ClientManageProcessor
getClientManageProcessor
()
{
return
clientManageProcessor
;
}
public
BrokerConfig
getBrokerConfig
()
{
return
brokerConfig
;
}
...
...
@@ -281,6 +318,15 @@ public class BrokerController {
this
.
pullThreadPoolQueue
,
new
ThreadFactoryImpl
(
"PullMessageThread_"
));
this
.
ackMessageExecutor
=
new
BrokerFixedThreadPoolExecutor
(
this
.
brokerConfig
.
getAckMessageThreadPoolNums
(),
this
.
brokerConfig
.
getAckMessageThreadPoolNums
(),
1000
*
60
,
TimeUnit
.
MILLISECONDS
,
this
.
ackThreadPoolQueue
,
new
ThreadFactoryImpl
(
"AckMessageThread_"
));
this
.
replyMessageExecutor
=
new
BrokerFixedThreadPoolExecutor
(
this
.
brokerConfig
.
getProcessReplyMessageThreadPoolNums
(),
this
.
brokerConfig
.
getProcessReplyMessageThreadPoolNums
(),
...
...
@@ -400,6 +446,10 @@ public class BrokerController {
}
},
1000
*
10
,
1000
*
60
,
TimeUnit
.
MILLISECONDS
);
this
.
loadBalanceExecutor
=
Executors
.
newFixedThreadPool
(
this
.
brokerConfig
.
getLoadBalanceProcessorThreadPoolNums
(),
new
ThreadFactoryImpl
(
"LoadBalanceProcessorThread_"
));
if
(
this
.
brokerConfig
.
getNamesrvAddr
()
!=
null
)
{
this
.
brokerOuterAPI
.
updateNameServerAddressList
(
this
.
brokerConfig
.
getNamesrvAddr
());
log
.
info
(
"Set user specified name server address: {}"
,
this
.
brokerConfig
.
getNamesrvAddr
());
...
...
@@ -547,23 +597,38 @@ public class BrokerController {
/**
* SendMessageProcessor
*/
SendMessageProcessor
sendProcessor
=
new
SendMessageProcessor
(
this
);
sendProcessor
.
registerSendMessageHook
(
sendMessageHookList
);
sendProcessor
.
registerConsumeMessageHook
(
consumeMessageHookList
);
this
.
remotingServer
.
registerProcessor
(
RequestCode
.
SEND_MESSAGE
,
sendProcessor
,
this
.
sendMessageExecutor
);
this
.
remotingServer
.
registerProcessor
(
RequestCode
.
SEND_MESSAGE_V2
,
sendProcessor
,
this
.
sendMessageExecutor
);
this
.
remotingServer
.
registerProcessor
(
RequestCode
.
SEND_BATCH_MESSAGE
,
sendProcessor
,
this
.
sendMessageExecutor
);
this
.
remotingServer
.
registerProcessor
(
RequestCode
.
CONSUMER_SEND_MSG_BACK
,
sendProcessor
,
this
.
sendMessageExecutor
);
this
.
fastRemotingServer
.
registerProcessor
(
RequestCode
.
SEND_MESSAGE
,
sendProcessor
,
this
.
sendMessageExecutor
);
this
.
fastRemotingServer
.
registerProcessor
(
RequestCode
.
SEND_MESSAGE_V2
,
sendProcessor
,
this
.
sendMessageExecutor
);
this
.
fastRemotingServer
.
registerProcessor
(
RequestCode
.
SEND_BATCH_MESSAGE
,
sendProcessor
,
this
.
sendMessageExecutor
);
this
.
fastRemotingServer
.
registerProcessor
(
RequestCode
.
CONSUMER_SEND_MSG_BACK
,
sendProcessor
,
this
.
sendMessageExecutor
);
send
Message
Processor
.
registerSendMessageHook
(
sendMessageHookList
);
send
Message
Processor
.
registerConsumeMessageHook
(
consumeMessageHookList
);
this
.
remotingServer
.
registerProcessor
(
RequestCode
.
SEND_MESSAGE
,
send
Message
Processor
,
this
.
sendMessageExecutor
);
this
.
remotingServer
.
registerProcessor
(
RequestCode
.
SEND_MESSAGE_V2
,
send
Message
Processor
,
this
.
sendMessageExecutor
);
this
.
remotingServer
.
registerProcessor
(
RequestCode
.
SEND_BATCH_MESSAGE
,
send
Message
Processor
,
this
.
sendMessageExecutor
);
this
.
remotingServer
.
registerProcessor
(
RequestCode
.
CONSUMER_SEND_MSG_BACK
,
send
Message
Processor
,
this
.
sendMessageExecutor
);
this
.
fastRemotingServer
.
registerProcessor
(
RequestCode
.
SEND_MESSAGE
,
send
Message
Processor
,
this
.
sendMessageExecutor
);
this
.
fastRemotingServer
.
registerProcessor
(
RequestCode
.
SEND_MESSAGE_V2
,
send
Message
Processor
,
this
.
sendMessageExecutor
);
this
.
fastRemotingServer
.
registerProcessor
(
RequestCode
.
SEND_BATCH_MESSAGE
,
send
Message
Processor
,
this
.
sendMessageExecutor
);
this
.
fastRemotingServer
.
registerProcessor
(
RequestCode
.
CONSUMER_SEND_MSG_BACK
,
send
Message
Processor
,
this
.
sendMessageExecutor
);
/**
* PullMessageProcessor
*/
this
.
remotingServer
.
registerProcessor
(
RequestCode
.
PULL_MESSAGE
,
this
.
pullMessageProcessor
,
this
.
pullMessageExecutor
);
this
.
pullMessageProcessor
.
registerConsumeMessageHook
(
consumeMessageHookList
);
/**
* PopMessageProcessor
*/
this
.
remotingServer
.
registerProcessor
(
RequestCode
.
POP_MESSAGE
,
this
.
popMessageProcessor
,
this
.
pullMessageExecutor
);
/**
* AckMessageProcessor
*/
this
.
remotingServer
.
registerProcessor
(
RequestCode
.
ACK_MESSAGE
,
this
.
ackMessageProcessor
,
this
.
ackMessageExecutor
);
this
.
fastRemotingServer
.
registerProcessor
(
RequestCode
.
ACK_MESSAGE
,
this
.
ackMessageProcessor
,
this
.
ackMessageExecutor
);
/**
* ChangeInvisibleTimeProcessor
*/
this
.
remotingServer
.
registerProcessor
(
RequestCode
.
CHANGE_MESSAGE_INVISIBLETIME
,
this
.
changeInvisibleTimeProcessor
,
this
.
ackMessageExecutor
);
this
.
fastRemotingServer
.
registerProcessor
(
RequestCode
.
CHANGE_MESSAGE_INVISIBLETIME
,
this
.
changeInvisibleTimeProcessor
,
this
.
ackMessageExecutor
);
/**
* ReplyMessageProcessor
...
...
@@ -589,14 +654,13 @@ public class BrokerController {
/**
* ClientManageProcessor
*/
ClientManageProcessor
clientProcessor
=
new
ClientManageProcessor
(
this
);
this
.
remotingServer
.
registerProcessor
(
RequestCode
.
HEART_BEAT
,
clientProcessor
,
this
.
heartbeatExecutor
);
this
.
remotingServer
.
registerProcessor
(
RequestCode
.
UNREGISTER_CLIENT
,
clientProcessor
,
this
.
clientManageExecutor
);
this
.
remotingServer
.
registerProcessor
(
RequestCode
.
CHECK_CLIENT_CONFIG
,
clientProcessor
,
this
.
clientManageExecutor
);
this
.
remotingServer
.
registerProcessor
(
RequestCode
.
HEART_BEAT
,
clientManageProcessor
,
this
.
heartbeatExecutor
);
this
.
remotingServer
.
registerProcessor
(
RequestCode
.
UNREGISTER_CLIENT
,
clientManageProcessor
,
this
.
clientManageExecutor
);
this
.
remotingServer
.
registerProcessor
(
RequestCode
.
CHECK_CLIENT_CONFIG
,
clientManageProcessor
,
this
.
clientManageExecutor
);
this
.
fastRemotingServer
.
registerProcessor
(
RequestCode
.
HEART_BEAT
,
clientProcessor
,
this
.
heartbeatExecutor
);
this
.
fastRemotingServer
.
registerProcessor
(
RequestCode
.
UNREGISTER_CLIENT
,
clientProcessor
,
this
.
clientManageExecutor
);
this
.
fastRemotingServer
.
registerProcessor
(
RequestCode
.
CHECK_CLIENT_CONFIG
,
clientProcessor
,
this
.
clientManageExecutor
);
this
.
fastRemotingServer
.
registerProcessor
(
RequestCode
.
HEART_BEAT
,
client
Manage
Processor
,
this
.
heartbeatExecutor
);
this
.
fastRemotingServer
.
registerProcessor
(
RequestCode
.
UNREGISTER_CLIENT
,
client
Manage
Processor
,
this
.
clientManageExecutor
);
this
.
fastRemotingServer
.
registerProcessor
(
RequestCode
.
CHECK_CLIENT_CONFIG
,
client
Manage
Processor
,
this
.
clientManageExecutor
);
/**
* ConsumerManageProcessor
...
...
@@ -610,6 +674,14 @@ public class BrokerController {
this
.
fastRemotingServer
.
registerProcessor
(
RequestCode
.
UPDATE_CONSUMER_OFFSET
,
consumerManageProcessor
,
this
.
consumerManageExecutor
);
this
.
fastRemotingServer
.
registerProcessor
(
RequestCode
.
QUERY_CONSUMER_OFFSET
,
consumerManageProcessor
,
this
.
consumerManageExecutor
);
/**
* QueryAssignmentProcessor
*/
this
.
remotingServer
.
registerProcessor
(
RequestCode
.
QUERY_ASSIGNMENT
,
queryAssignmentProcessor
,
loadBalanceExecutor
);
this
.
fastRemotingServer
.
registerProcessor
(
RequestCode
.
QUERY_ASSIGNMENT
,
queryAssignmentProcessor
,
loadBalanceExecutor
);
this
.
remotingServer
.
registerProcessor
(
RequestCode
.
SET_MESSAGE_REQUEST_MODE
,
queryAssignmentProcessor
,
loadBalanceExecutor
);
this
.
fastRemotingServer
.
registerProcessor
(
RequestCode
.
SET_MESSAGE_REQUEST_MODE
,
queryAssignmentProcessor
,
loadBalanceExecutor
);
/**
* EndTransactionProcessor
*/
...
...
@@ -713,6 +785,10 @@ public class BrokerController {
return
consumerFilterManager
;
}
public
ConsumerOrderInfoManager
getConsumerOrderInfoManager
()
{
return
consumerOrderInfoManager
;
}
public
ConsumerOffsetManager
getConsumerOffsetManager
()
{
return
consumerOffsetManager
;
}
...
...
@@ -741,6 +817,10 @@ public class BrokerController {
return
subscriptionGroupManager
;
}
public
PopMessageProcessor
getPopMessageProcessor
()
{
return
popMessageProcessor
;
}
public
void
shutdown
()
{
if
(
this
.
brokerStatsManager
!=
null
)
{
this
.
brokerStatsManager
.
shutdown
();
...
...
@@ -824,6 +904,11 @@ public class BrokerController {
this
.
consumerManageExecutor
.
shutdown
();
}
{
this
.
popMessageProcessor
.
getPopBufferMergeService
().
shutdown
();
this
.
ackMessageProcessor
.
shutdownPopReviveService
();
}
if
(
this
.
fileWatchService
!=
null
)
{
this
.
fileWatchService
.
shutdown
();
}
...
...
@@ -849,6 +934,8 @@ public class BrokerController {
}
public
void
start
()
throws
Exception
{
this
.
shouldStartTime
=
System
.
currentTimeMillis
();
if
(
this
.
messageStore
!=
null
)
{
this
.
messageStore
.
start
();
}
...
...
@@ -857,6 +944,17 @@ public class BrokerController {
this
.
remotingServer
.
start
();
}
{
this
.
popMessageProcessor
.
getPopLongPollingService
().
start
();
this
.
popMessageProcessor
.
getPopBufferMergeService
().
start
();
this
.
popMessageProcessor
.
getQueueLockManager
().
start
();
this
.
ackMessageProcessor
.
startPopReviveService
();
}
{
assignmentManager
.
start
();
}
if
(
this
.
fastRemotingServer
!=
null
)
{
this
.
fastRemotingServer
.
start
();
}
...
...
@@ -1243,4 +1341,20 @@ public class BrokerController {
public
ExecutorService
getSendMessageExecutor
()
{
return
sendMessageExecutor
;
}
public
long
getShouldStartTime
()
{
return
shouldStartTime
;
}
public
AssignmentManager
getAssignmentManager
()
{
return
assignmentManager
;
}
public
SendMessageProcessor
getSendMessageProcessor
()
{
return
sendMessageProcessor
;
}
public
QueryAssignmentProcessor
getQueryAssignmentProcessor
()
{
return
queryAssignmentProcessor
;
}
}
broker/src/main/java/org/apache/rocketmq/broker/BrokerPathConfigHelper.java
浏览文件 @
ced6b023
...
...
@@ -39,6 +39,10 @@ public class BrokerPathConfigHelper {
return
rootDir
+
File
.
separator
+
"config"
+
File
.
separator
+
"consumerOffset.json"
;
}
public
static
String
getConsumerOrderInfoPath
(
final
String
rootDir
)
{
return
rootDir
+
File
.
separator
+
"config"
+
File
.
separator
+
"consumerOrderInfo.json"
;
}
public
static
String
getSubscriptionGroupPath
(
final
String
rootDir
)
{
return
rootDir
+
File
.
separator
+
"config"
+
File
.
separator
+
"subscriptionGroup.json"
;
}
...
...
@@ -46,4 +50,8 @@ public class BrokerPathConfigHelper {
public
static
String
getConsumerFilterPath
(
final
String
rootDir
)
{
return
rootDir
+
File
.
separator
+
"config"
+
File
.
separator
+
"consumerFilter.json"
;
}
public
static
String
getMessageRequestModePath
(
final
String
rootDir
)
{
return
rootDir
+
File
.
separator
+
"config"
+
File
.
separator
+
"messageRequestMode.json"
;
}
}
broker/src/main/java/org/apache/rocketmq/broker/loadbalance/AssignmentManager.java
0 → 100644
浏览文件 @
ced6b023
/*
* 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.loadbalance
;
import
com.google.common.collect.Lists
;
import
java.util.HashSet
;
import
java.util.List
;
import
java.util.Set
;
import
java.util.concurrent.ConcurrentHashMap
;
import
java.util.concurrent.Executors
;
import
java.util.concurrent.ScheduledExecutorService
;
import
java.util.concurrent.TimeUnit
;
import
java.util.concurrent.locks.Lock
;
import
java.util.concurrent.locks.ReentrantLock
;
import
org.apache.rocketmq.broker.BrokerController
;
import
org.apache.rocketmq.broker.out.BrokerOuterAPI
;
import
org.apache.rocketmq.client.exception.MQBrokerException
;
import
org.apache.rocketmq.client.impl.factory.MQClientInstance
;
import
org.apache.rocketmq.common.MixAll
;
import
org.apache.rocketmq.common.ThreadFactoryImpl
;
import
org.apache.rocketmq.common.constant.LoggerName
;
import
org.apache.rocketmq.common.message.MessageQueue
;
import
org.apache.rocketmq.common.protocol.ResponseCode
;
import
org.apache.rocketmq.common.protocol.route.TopicRouteData
;
import
org.apache.rocketmq.common.topic.TopicValidator
;
import
org.apache.rocketmq.logging.InternalLogger
;
import
org.apache.rocketmq.logging.InternalLoggerFactory
;
public
class
AssignmentManager
{
private
static
final
InternalLogger
log
=
InternalLoggerFactory
.
getLogger
(
LoggerName
.
BROKER_LOGGER_NAME
);
private
transient
BrokerController
brokerController
;
private
final
static
long
LOCK_TIMEOUT_MILLIS
=
3000
;
private
final
Lock
lockNamesrv
=
new
ReentrantLock
();
private
final
BrokerOuterAPI
mQClientAPIImpl
;
private
final
ConcurrentHashMap
<
String
,
Set
<
MessageQueue
>>
topicSubscribeInfoTable
=
new
ConcurrentHashMap
<
String
,
Set
<
MessageQueue
>>();
private
ScheduledExecutorService
scheduledExecutorService
=
Executors
.
newSingleThreadScheduledExecutor
(
new
ThreadFactoryImpl
(
"LoadBalanceManagerScheduledThread"
));
private
static
final
List
<
String
>
IGNORE_ROUTE_TOPICS
=
Lists
.
newArrayList
(
TopicValidator
.
SYSTEM_TOPIC_PREFIX
,
MixAll
.
CID_RMQ_SYS_PREFIX
,
MixAll
.
DEFAULT_CONSUMER_GROUP
,
MixAll
.
TOOLS_CONSUMER_GROUP
,
MixAll
.
FILTERSRV_CONSUMER_GROUP
,
MixAll
.
MONITOR_CONSUMER_GROUP
,
MixAll
.
ONS_HTTP_PROXY_GROUP
,
MixAll
.
CID_ONSAPI_PERMISSION_GROUP
,
MixAll
.
CID_ONSAPI_OWNER_GROUP
,
MixAll
.
CID_ONSAPI_PULL_GROUP
);
private
final
List
<
String
>
ignoreRouteTopics
=
Lists
.
newArrayList
(
IGNORE_ROUTE_TOPICS
);
public
AssignmentManager
(
BrokerController
brokerController
)
{
this
.
brokerController
=
brokerController
;
this
.
mQClientAPIImpl
=
brokerController
.
getBrokerOuterAPI
();
ignoreRouteTopics
.
add
(
brokerController
.
getBrokerConfig
().
getBrokerClusterName
());
ignoreRouteTopics
.
add
(
brokerController
.
getBrokerConfig
().
getBrokerName
());
}
public
void
start
()
{
this
.
scheduledExecutorService
.
scheduleAtFixedRate
(
new
Runnable
()
{
@Override
public
void
run
()
{
try
{
updateTopicRouteInfoFromNameServer
();
}
catch
(
Exception
e
)
{
log
.
error
(
"ScheduledTask: failed to pull TopicRouteData from NameServer"
,
e
);
}
}
},
13000
,
this
.
brokerController
.
getBrokerConfig
().
getLoadBalancePollNameServerInterval
(),
TimeUnit
.
MILLISECONDS
);
}
public
void
updateTopicRouteInfoFromNameServer
()
{
Set
<
String
>
topicList
=
new
HashSet
<>(
brokerController
.
getTopicConfigManager
().
getTopicConfigTable
().
keySet
());
LOOP:
for
(
String
topic
:
topicList
)
{
for
(
String
keyword
:
ignoreRouteTopics
)
{
if
(
topic
.
contains
(
keyword
))
{
continue
LOOP
;
}
}
this
.
updateTopicRouteInfoFromNameServer
(
topic
);
}
}
public
boolean
updateTopicRouteInfoFromNameServer
(
final
String
topic
)
{
try
{
TopicRouteData
topicRouteData
=
this
.
mQClientAPIImpl
.
getTopicRouteInfoFromNameServer
(
topic
,
1000
*
3
);
if
(
topicRouteData
!=
null
)
{
Set
<
MessageQueue
>
newSubscribeInfo
=
MQClientInstance
.
topicRouteData2TopicSubscribeInfo
(
topic
,
topicRouteData
);
Set
<
MessageQueue
>
oldSubscribeInfo
=
topicSubscribeInfoTable
.
get
(
topic
);
boolean
changed
=
!
newSubscribeInfo
.
equals
(
oldSubscribeInfo
);
if
(
changed
)
{
log
.
info
(
"the topic[{}] subscribe message queue changed, old[{}] ,new[{}]"
,
topic
,
oldSubscribeInfo
,
newSubscribeInfo
);
topicSubscribeInfoTable
.
put
(
topic
,
newSubscribeInfo
);
return
true
;
}
}
else
{
log
.
warn
(
"updateTopicRouteInfoFromNameServer, getTopicRouteInfoFromNameServer return null, Topic: {}"
,
topic
);
}
}
catch
(
Exception
e
)
{
if
(!
topic
.
startsWith
(
MixAll
.
RETRY_GROUP_TOPIC_PREFIX
))
{
log
.
warn
(
"updateTopicRouteInfoFromNameServer Exception"
,
e
);
if
(
e
instanceof
MQBrokerException
&&
ResponseCode
.
TOPIC_NOT_EXIST
==
((
MQBrokerException
)
e
).
getResponseCode
())
{
// clean no used topic
cleanNoneRouteTopic
(
topic
);
}
}
}
return
false
;
}
private
void
cleanNoneRouteTopic
(
String
topic
)
{
// clean no used topic
topicSubscribeInfoTable
.
remove
(
topic
);
}
public
Set
<
MessageQueue
>
getTopicSubscribeInfo
(
String
topic
)
{
return
topicSubscribeInfoTable
.
get
(
topic
);
}
}
broker/src/main/java/org/apache/rocketmq/broker/loadbalance/MessageRequestModeManager.java
0 → 100644
浏览文件 @
ced6b023
/*
* 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.loadbalance
;
import
java.util.concurrent.ConcurrentHashMap
;
import
org.apache.rocketmq.broker.BrokerController
;
import
org.apache.rocketmq.broker.BrokerPathConfigHelper
;
import
org.apache.rocketmq.common.ConfigManager
;
import
org.apache.rocketmq.common.protocol.body.SetMessageRequestModeRequestBody
;
import
org.apache.rocketmq.remoting.protocol.RemotingSerializable
;
public
class
MessageRequestModeManager
extends
ConfigManager
{
private
BrokerController
brokerController
;
private
ConcurrentHashMap
<
String
,
ConcurrentHashMap
<
String
,
SetMessageRequestModeRequestBody
>>
messageRequestModeMap
=
new
ConcurrentHashMap
<
String
,
ConcurrentHashMap
<
String
,
SetMessageRequestModeRequestBody
>>();
public
MessageRequestModeManager
()
{
// empty construct for decode
}
public
MessageRequestModeManager
(
BrokerController
brokerController
)
{
this
.
brokerController
=
brokerController
;
}
public
void
setMessageRequestMode
(
String
topic
,
String
consumerGroup
,
SetMessageRequestModeRequestBody
requestBody
)
{
ConcurrentHashMap
<
String
,
SetMessageRequestModeRequestBody
>
consumerGroup2ModeMap
=
messageRequestModeMap
.
get
(
topic
);
if
(
consumerGroup2ModeMap
==
null
)
{
consumerGroup2ModeMap
=
new
ConcurrentHashMap
<
String
,
SetMessageRequestModeRequestBody
>();
ConcurrentHashMap
<
String
,
SetMessageRequestModeRequestBody
>
pre
=
messageRequestModeMap
.
putIfAbsent
(
topic
,
consumerGroup2ModeMap
);
if
(
pre
!=
null
)
{
consumerGroup2ModeMap
=
pre
;
}
}
consumerGroup2ModeMap
.
put
(
consumerGroup
,
requestBody
);
}
public
SetMessageRequestModeRequestBody
getMessageRequestMode
(
String
topic
,
String
consumerGroup
)
{
ConcurrentHashMap
<
String
,
SetMessageRequestModeRequestBody
>
consumerGroup2ModeMap
=
messageRequestModeMap
.
get
(
topic
);
if
(
consumerGroup2ModeMap
!=
null
)
{
return
consumerGroup2ModeMap
.
get
(
consumerGroup
);
}
return
null
;
}
public
ConcurrentHashMap
<
String
,
ConcurrentHashMap
<
String
,
SetMessageRequestModeRequestBody
>>
getMessageRequestModeMap
()
{
return
this
.
messageRequestModeMap
;
}
public
void
setMessageRequestModeMap
(
ConcurrentHashMap
<
String
,
ConcurrentHashMap
<
String
,
SetMessageRequestModeRequestBody
>>
messageRequestModeMap
)
{
this
.
messageRequestModeMap
=
messageRequestModeMap
;
}
@Override
public
String
encode
()
{
return
this
.
encode
(
false
);
}
@Override
public
String
configFilePath
()
{
return
BrokerPathConfigHelper
.
getMessageRequestModePath
(
this
.
brokerController
.
getMessageStoreConfig
().
getStorePathRootDir
());
}
@Override
public
void
decode
(
String
jsonString
)
{
if
(
jsonString
!=
null
)
{
MessageRequestModeManager
obj
=
RemotingSerializable
.
fromJson
(
jsonString
,
MessageRequestModeManager
.
class
);
if
(
obj
!=
null
)
{
this
.
messageRequestModeMap
=
obj
.
messageRequestModeMap
;
}
}
}
@Override
public
String
encode
(
boolean
prettyFormat
)
{
return
RemotingSerializable
.
toJson
(
this
,
prettyFormat
);
}
}
broker/src/main/java/org/apache/rocketmq/broker/longpolling/NotifyMessageArrivingListener.java
浏览文件 @
ced6b023
...
...
@@ -17,15 +17,17 @@
package
org.apache.rocketmq.broker.longpolling
;
import
org.apache.rocketmq.store.MessageArrivingListener
;
import
java.util.Map
;
import
org.apache.rocketmq.broker.processor.PopMessageProcessor
;
import
org.apache.rocketmq.store.MessageArrivingListener
;
public
class
NotifyMessageArrivingListener
implements
MessageArrivingListener
{
private
final
PullRequestHoldService
pullRequestHoldService
;
private
final
PopMessageProcessor
popMessageProcessor
;
public
NotifyMessageArrivingListener
(
final
PullRequestHoldService
pullRequestHoldService
)
{
public
NotifyMessageArrivingListener
(
final
PullRequestHoldService
pullRequestHoldService
,
final
PopMessageProcessor
popMessageProcessor
)
{
this
.
pullRequestHoldService
=
pullRequestHoldService
;
this
.
popMessageProcessor
=
popMessageProcessor
;
}
@Override
...
...
@@ -33,5 +35,6 @@ public class NotifyMessageArrivingListener implements MessageArrivingListener {
long
msgStoreTime
,
byte
[]
filterBitMap
,
Map
<
String
,
String
>
properties
)
{
this
.
pullRequestHoldService
.
notifyMessageArriving
(
topic
,
queueId
,
logicOffset
,
tagsCode
,
msgStoreTime
,
filterBitMap
,
properties
);
this
.
popMessageProcessor
.
notifyMessageArriving
(
topic
,
queueId
);
}
}
broker/src/main/java/org/apache/rocketmq/broker/longpolling/PopRequest.java
0 → 100644
浏览文件 @
ced6b023
/*
* 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.longpolling
;
import
io.netty.channel.Channel
;
import
java.util.Comparator
;
import
java.util.concurrent.atomic.AtomicBoolean
;
import
java.util.concurrent.atomic.AtomicLong
;
import
org.apache.rocketmq.remoting.protocol.RemotingCommand
;
public
class
PopRequest
{
private
static
final
AtomicLong
COUNTER
=
new
AtomicLong
(
Long
.
MIN_VALUE
);
private
RemotingCommand
remotingCommand
;
private
Channel
channel
;
private
long
expired
;
private
AtomicBoolean
complete
=
new
AtomicBoolean
(
false
);
private
final
long
op
=
COUNTER
.
getAndIncrement
();
public
PopRequest
(
RemotingCommand
remotingCommand
,
Channel
channel
,
long
expired
)
{
this
.
channel
=
channel
;
this
.
remotingCommand
=
remotingCommand
;
this
.
expired
=
expired
;
}
public
Channel
getChannel
()
{
return
channel
;
}
public
RemotingCommand
getRemotingCommand
()
{
return
remotingCommand
;
}
public
boolean
isTimeout
()
{
return
System
.
currentTimeMillis
()
>
(
expired
-
50
);
}
public
boolean
complete
()
{
return
complete
.
compareAndSet
(
false
,
true
);
}
public
long
getExpired
()
{
return
expired
;
}
@Override
public
String
toString
()
{
final
StringBuilder
sb
=
new
StringBuilder
(
"PopRequest{"
);
sb
.
append
(
"cmd="
).
append
(
remotingCommand
);
sb
.
append
(
", channel="
).
append
(
channel
);
sb
.
append
(
", expired="
).
append
(
expired
);
sb
.
append
(
", complete="
).
append
(
complete
);
sb
.
append
(
", op="
).
append
(
op
);
sb
.
append
(
'}'
);
return
sb
.
toString
();
}
public
static
final
Comparator
<
PopRequest
>
COMPARATOR
=
new
Comparator
<
PopRequest
>()
{
@Override
public
int
compare
(
PopRequest
o1
,
PopRequest
o2
)
{
int
ret
=
(
int
)
(
o1
.
getExpired
()
-
o2
.
getExpired
());
if
(
ret
!=
0
)
{
return
ret
;
}
ret
=
(
int
)
(
o1
.
op
-
o2
.
op
);
if
(
ret
!=
0
)
{
return
ret
;
}
return
-
1
;
}
};
}
broker/src/main/java/org/apache/rocketmq/broker/offset/ConsumerOrderInfoManager.java
0 → 100644
浏览文件 @
ced6b023
/*
* 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.offset
;
import
com.alibaba.fastjson.annotation.JSONField
;
import
java.util.ArrayList
;
import
java.util.Iterator
;
import
java.util.List
;
import
java.util.Map
;
import
java.util.concurrent.ConcurrentHashMap
;
import
org.apache.rocketmq.broker.BrokerController
;
import
org.apache.rocketmq.broker.BrokerPathConfigHelper
;
import
org.apache.rocketmq.common.ConfigManager
;
import
org.apache.rocketmq.common.TopicConfig
;
import
org.apache.rocketmq.common.constant.LoggerName
;
import
org.apache.rocketmq.logging.InternalLogger
;
import
org.apache.rocketmq.logging.InternalLoggerFactory
;
import
org.apache.rocketmq.remoting.protocol.RemotingSerializable
;
public
class
ConsumerOrderInfoManager
extends
ConfigManager
{
private
static
final
InternalLogger
log
=
InternalLoggerFactory
.
getLogger
(
LoggerName
.
BROKER_LOGGER_NAME
);
private
static
final
String
TOPIC_GROUP_SEPARATOR
=
"@"
;
private
static
final
long
CLEAN_SPAN_FROM_LAST
=
24
*
3600
*
1000
;
private
ConcurrentHashMap
<
String
/* topic@group*/
,
ConcurrentHashMap
<
Integer
/*queueId*/
,
OrderInfo
>>
table
=
new
ConcurrentHashMap
<>(
128
);
private
transient
BrokerController
brokerController
;
public
ConsumerOrderInfoManager
()
{
}
public
ConsumerOrderInfoManager
(
BrokerController
brokerController
)
{
this
.
brokerController
=
brokerController
;
}
public
ConcurrentHashMap
<
String
,
ConcurrentHashMap
<
Integer
,
OrderInfo
>>
getTable
()
{
return
table
;
}
public
void
setTable
(
ConcurrentHashMap
<
String
,
ConcurrentHashMap
<
Integer
,
OrderInfo
>>
table
)
{
this
.
table
=
table
;
}
/**
* not thread safe.
*
* @param topic
* @param group
* @param queueId
* @param msgOffsetList
*/
public
int
update
(
String
topic
,
String
group
,
int
queueId
,
List
<
Long
>
msgOffsetList
)
{
String
key
=
topic
+
TOPIC_GROUP_SEPARATOR
+
group
;
ConcurrentHashMap
<
Integer
/*queueId*/
,
OrderInfo
>
qs
=
table
.
get
(
key
);
if
(
qs
==
null
)
{
qs
=
new
ConcurrentHashMap
<>(
16
);
ConcurrentHashMap
<
Integer
/*queueId*/
,
OrderInfo
>
old
=
table
.
putIfAbsent
(
key
,
qs
);
if
(
old
!=
null
)
{
qs
=
old
;
}
}
OrderInfo
orderInfo
=
qs
.
get
(
queueId
);
// start is same.
List
<
Long
>
simple
=
OrderInfo
.
simpleO
(
msgOffsetList
);
if
(
orderInfo
!=
null
&&
simple
.
get
(
0
).
equals
(
orderInfo
.
getOffsetList
().
get
(
0
)))
{
if
(
simple
.
equals
(
orderInfo
.
getOffsetList
()))
{
orderInfo
.
setConsumedCount
(
orderInfo
.
getConsumedCount
()
+
1
);
}
else
{
// reset, because msgs are changed.
orderInfo
.
setConsumedCount
(
0
);
}
orderInfo
.
setLastConsumeTimestamp
(
System
.
currentTimeMillis
());
orderInfo
.
setOffsetList
(
simple
);
orderInfo
.
setCommitOffsetBit
(
0
);
}
else
{
orderInfo
=
new
OrderInfo
();
orderInfo
.
setOffsetList
(
simple
);
orderInfo
.
setLastConsumeTimestamp
(
System
.
currentTimeMillis
());
orderInfo
.
setConsumedCount
(
0
);
orderInfo
.
setCommitOffsetBit
(
0
);
qs
.
put
(
queueId
,
orderInfo
);
}
return
orderInfo
.
getConsumedCount
();
}
public
boolean
checkBlock
(
String
topic
,
String
group
,
int
queueId
,
long
invisibleTime
)
{
String
key
=
topic
+
TOPIC_GROUP_SEPARATOR
+
group
;
ConcurrentHashMap
<
Integer
/*queueId*/
,
OrderInfo
>
qs
=
table
.
get
(
key
);
if
(
qs
==
null
)
{
qs
=
new
ConcurrentHashMap
<>(
16
);
ConcurrentHashMap
<
Integer
/*queueId*/
,
OrderInfo
>
old
=
table
.
putIfAbsent
(
key
,
qs
);
if
(
old
!=
null
)
{
qs
=
old
;
}
}
OrderInfo
orderInfo
=
qs
.
get
(
queueId
);
if
(
orderInfo
==
null
)
{
return
false
;
}
boolean
isBlock
=
System
.
currentTimeMillis
()
-
orderInfo
.
getLastConsumeTimestamp
()
<
invisibleTime
;
return
isBlock
&&
!
orderInfo
.
isDone
();
}
/**
* @param topic
* @param group
* @param queueId
* @param offset
* @return -1 : illegal, -2 : no need commit, >= 0 : commit
*/
public
long
commitAndNext
(
String
topic
,
String
group
,
int
queueId
,
long
offset
)
{
String
key
=
topic
+
TOPIC_GROUP_SEPARATOR
+
group
;
ConcurrentHashMap
<
Integer
/*queueId*/
,
OrderInfo
>
qs
=
table
.
get
(
key
);
if
(
qs
==
null
)
{
return
offset
+
1
;
}
OrderInfo
orderInfo
=
qs
.
get
(
queueId
);
if
(
orderInfo
==
null
)
{
log
.
warn
(
"OrderInfo is null, {}, {}, {}"
,
key
,
offset
,
orderInfo
);
return
offset
+
1
;
}
List
<
Long
>
offsetList
=
orderInfo
.
getOffsetList
();
if
(
offsetList
==
null
||
offsetList
.
isEmpty
())
{
log
.
warn
(
"OrderInfo is empty, {}, {}, {}"
,
key
,
offset
,
orderInfo
);
return
-
1
;
}
Long
first
=
offsetList
.
get
(
0
);
int
i
=
0
,
size
=
offsetList
.
size
();
for
(;
i
<
size
;
i
++)
{
long
temp
;
if
(
i
==
0
)
{
temp
=
first
;
}
else
{
temp
=
first
+
offsetList
.
get
(
i
);
}
if
(
offset
==
temp
)
{
break
;
}
}
// not found
if
(
i
>=
size
)
{
log
.
warn
(
"OrderInfo not found commit offset, {}, {}, {}"
,
key
,
offset
,
orderInfo
);
return
-
1
;
}
//set bit
orderInfo
.
setCommitOffsetBit
(
orderInfo
.
getCommitOffsetBit
()
|
(
1L
<<
i
));
if
(
orderInfo
.
isDone
())
{
if
(
size
==
1
)
{
return
offsetList
.
get
(
0
)
+
1
;
}
else
{
return
offsetList
.
get
(
size
-
1
)
+
first
+
1
;
}
}
return
-
2
;
}
public
OrderInfo
get
(
String
topic
,
String
group
,
int
queueId
)
{
String
key
=
topic
+
TOPIC_GROUP_SEPARATOR
+
group
;
ConcurrentHashMap
<
Integer
/*queueId*/
,
OrderInfo
>
qs
=
table
.
get
(
key
);
if
(
qs
==
null
)
{
return
null
;
}
return
qs
.
get
(
queueId
);
}
public
int
getConsumeCount
(
String
topic
,
String
group
,
int
queueId
)
{
OrderInfo
orderInfo
=
get
(
topic
,
group
,
queueId
);
return
orderInfo
==
null
?
0
:
orderInfo
.
getConsumedCount
();
}
private
void
autoClean
()
{
if
(
brokerController
==
null
)
{
return
;
}
Iterator
<
Map
.
Entry
<
String
/* topic@group*/
,
ConcurrentHashMap
<
Integer
/*queueId*/
,
OrderInfo
>>>
iterator
=
this
.
table
.
entrySet
().
iterator
();
while
(
iterator
.
hasNext
())
{
Map
.
Entry
<
String
/* topic@group*/
,
ConcurrentHashMap
<
Integer
/*queueId*/
,
OrderInfo
>>
entry
=
iterator
.
next
();
String
topicAtGroup
=
entry
.
getKey
();
ConcurrentHashMap
<
Integer
/*queueId*/
,
OrderInfo
>
qs
=
entry
.
getValue
();
String
[]
arrays
=
topicAtGroup
.
split
(
TOPIC_GROUP_SEPARATOR
);
if
(
arrays
.
length
!=
2
)
{
continue
;
}
String
topic
=
arrays
[
0
];
String
group
=
arrays
[
1
];
TopicConfig
topicConfig
=
this
.
brokerController
.
getTopicConfigManager
().
selectTopicConfig
(
topic
);
if
(
topicConfig
==
null
)
{
iterator
.
remove
();
log
.
info
(
"Topic not exist, Clean order info, {}:{}"
,
topicAtGroup
,
qs
);
continue
;
}
if
(
this
.
brokerController
.
getSubscriptionGroupManager
().
getSubscriptionGroupTable
().
get
(
group
)
==
null
)
{
iterator
.
remove
();
log
.
info
(
"Group not exist, Clean order info, {}:{}"
,
topicAtGroup
,
qs
);
continue
;
}
if
(
qs
.
isEmpty
())
{
iterator
.
remove
();
log
.
info
(
"Order table is empty, Clean order info, {}:{}"
,
topicAtGroup
,
qs
);
continue
;
}
Iterator
<
Map
.
Entry
<
Integer
/*queueId*/
,
OrderInfo
>>
qsIterator
=
qs
.
entrySet
().
iterator
();
while
(
qsIterator
.
hasNext
())
{
Map
.
Entry
<
Integer
/*queueId*/
,
OrderInfo
>
qsEntry
=
qsIterator
.
next
();
if
(
qsEntry
.
getKey
()
>=
topicConfig
.
getReadQueueNums
())
{
qsIterator
.
remove
();
log
.
info
(
"Queue not exist, Clean order info, {}:{}, {}"
,
topicAtGroup
,
entry
.
getValue
(),
topicConfig
);
continue
;
}
if
(
System
.
currentTimeMillis
()
-
qsEntry
.
getValue
().
getLastConsumeTimestamp
()
>
CLEAN_SPAN_FROM_LAST
)
{
qsIterator
.
remove
();
log
.
info
(
"Not consume long time, Clean order info, {}:{}, {}"
,
topicAtGroup
,
entry
.
getValue
(),
topicConfig
);
continue
;
}
}
}
}
@Override
public
String
encode
()
{
return
this
.
encode
(
false
);
}
@Override
public
String
configFilePath
()
{
if
(
brokerController
!=
null
)
{
return
BrokerPathConfigHelper
.
getConsumerOrderInfoPath
(
this
.
brokerController
.
getMessageStoreConfig
().
getStorePathRootDir
());
}
else
{
return
BrokerPathConfigHelper
.
getConsumerOrderInfoPath
(
"~"
);
}
}
@Override
public
void
decode
(
String
jsonString
)
{
if
(
jsonString
!=
null
)
{
ConsumerOrderInfoManager
obj
=
RemotingSerializable
.
fromJson
(
jsonString
,
ConsumerOrderInfoManager
.
class
);
if
(
obj
!=
null
)
{
this
.
table
=
obj
.
table
;
}
}
}
@Override
public
String
encode
(
boolean
prettyFormat
)
{
this
.
autoClean
();
StringBuilder
stringBuilder
=
new
StringBuilder
();
stringBuilder
.
append
(
"{\n"
).
append
(
"\t\"table\":{"
);
Iterator
<
Map
.
Entry
<
String
/* topic@group*/
,
ConcurrentHashMap
<
Integer
/*queueId*/
,
OrderInfo
>>>
iterator
=
this
.
table
.
entrySet
().
iterator
();
int
count1
=
0
;
while
(
iterator
.
hasNext
())
{
Map
.
Entry
<
String
/* topic@group*/
,
ConcurrentHashMap
<
Integer
/*queueId*/
,
OrderInfo
>>
entry
=
iterator
.
next
();
if
(
count1
>
0
)
{
stringBuilder
.
append
(
","
);
}
stringBuilder
.
append
(
"\n\t\t\""
).
append
(
entry
.
getKey
()).
append
(
"\":{"
);
Iterator
<
Map
.
Entry
<
Integer
/*queueId*/
,
OrderInfo
>>
qsIterator
=
entry
.
getValue
().
entrySet
().
iterator
();
int
count2
=
0
;
while
(
qsIterator
.
hasNext
())
{
Map
.
Entry
<
Integer
/*queueId*/
,
OrderInfo
>
qsEntry
=
qsIterator
.
next
();
if
(
count2
>
0
)
{
stringBuilder
.
append
(
","
);
}
stringBuilder
.
append
(
"\n\t\t\t"
).
append
(
qsEntry
.
getKey
()).
append
(
":"
)
.
append
(
qsEntry
.
getValue
().
encode
());
count2
++;
}
stringBuilder
.
append
(
"\n\t\t}"
);
count1
++;
}
stringBuilder
.
append
(
"\n\t}"
).
append
(
"\n}"
);
return
stringBuilder
.
toString
();
}
public
static
class
OrderInfo
{
/**
* offset
*/
private
List
<
Long
>
offsetList
;
/**
* consumed count
*/
private
int
consumedCount
;
/**
* last consume timestamp
*/
private
long
lastConsumeTimestamp
;
/**
* commit offset bit
*/
private
long
commitOffsetBit
;
public
OrderInfo
()
{
}
public
List
<
Long
>
getOffsetList
()
{
return
offsetList
;
}
public
void
setOffsetList
(
List
<
Long
>
offsetList
)
{
this
.
offsetList
=
offsetList
;
}
public
static
List
<
Long
>
simpleO
(
List
<
Long
>
offsetList
)
{
List
<
Long
>
simple
=
new
ArrayList
<>();
if
(
offsetList
.
size
()
==
1
)
{
simple
.
addAll
(
offsetList
);
return
simple
;
}
Long
first
=
offsetList
.
get
(
0
);
simple
.
add
(
first
);
for
(
int
i
=
1
;
i
<
offsetList
.
size
();
i
++)
{
simple
.
add
(
offsetList
.
get
(
i
)
-
first
);
}
return
simple
;
}
public
int
getConsumedCount
()
{
return
consumedCount
;
}
public
void
setConsumedCount
(
int
consumedCount
)
{
this
.
consumedCount
=
consumedCount
;
}
public
long
getLastConsumeTimestamp
()
{
return
lastConsumeTimestamp
;
}
public
void
setLastConsumeTimestamp
(
long
lastConsumeTimestamp
)
{
this
.
lastConsumeTimestamp
=
lastConsumeTimestamp
;
}
public
long
getCommitOffsetBit
()
{
return
commitOffsetBit
;
}
public
void
setCommitOffsetBit
(
long
commitOffsetBit
)
{
this
.
commitOffsetBit
=
commitOffsetBit
;
}
@JSONField
(
serialize
=
false
,
deserialize
=
false
)
public
boolean
isDone
()
{
if
(
offsetList
==
null
||
offsetList
.
isEmpty
())
{
return
true
;
}
int
num
=
offsetList
.
size
();
for
(
byte
i
=
0
;
i
<
num
;
i
++)
{
if
((
commitOffsetBit
&
(
1L
<<
i
))
==
0
)
{
return
false
;
}
}
return
true
;
}
@JSONField
(
serialize
=
false
,
deserialize
=
false
)
public
String
encode
()
{
StringBuilder
sb
=
new
StringBuilder
();
sb
.
append
(
"{"
).
append
(
"\"c\":"
).
append
(
getConsumedCount
());
sb
.
append
(
","
).
append
(
"\"cm\":"
).
append
(
getCommitOffsetBit
());
sb
.
append
(
","
).
append
(
"\"l\":"
).
append
(
getLastConsumeTimestamp
());
sb
.
append
(
","
).
append
(
"\"o\":["
);
if
(
getOffsetList
()
!=
null
)
{
for
(
int
i
=
0
;
i
<
getOffsetList
().
size
();
i
++)
{
sb
.
append
(
getOffsetList
().
get
(
i
));
if
(
i
<
getOffsetList
().
size
()
-
1
)
{
sb
.
append
(
","
);
}
}
}
sb
.
append
(
"]"
).
append
(
"}"
);
return
sb
.
toString
();
}
@Override
public
String
toString
()
{
final
StringBuilder
sb
=
new
StringBuilder
(
"OrderInfo"
);
sb
.
append
(
"@"
).
append
(
this
.
hashCode
());
sb
.
append
(
"{offsetList="
).
append
(
offsetList
);
sb
.
append
(
", consumedCount="
).
append
(
consumedCount
);
sb
.
append
(
", lastConsumeTimestamp="
).
append
(
lastConsumeTimestamp
);
sb
.
append
(
", commitOffsetBit="
).
append
(
commitOffsetBit
);
sb
.
append
(
", isDone="
).
append
(
isDone
());
sb
.
append
(
'}'
);
return
sb
.
toString
();
}
}
}
broker/src/main/java/org/apache/rocketmq/broker/out/BrokerOuterAPI.java
浏览文件 @
ced6b023
...
...
@@ -30,8 +30,6 @@ import org.apache.rocketmq.common.MixAll;
import
org.apache.rocketmq.common.ThreadFactoryImpl
;
import
org.apache.rocketmq.common.UtilAll
;
import
org.apache.rocketmq.common.constant.LoggerName
;
import
org.apache.rocketmq.logging.InternalLogger
;
import
org.apache.rocketmq.logging.InternalLoggerFactory
;
import
org.apache.rocketmq.common.namesrv.RegisterBrokerResult
;
import
org.apache.rocketmq.common.namesrv.TopAddressing
;
import
org.apache.rocketmq.common.protocol.RequestCode
;
...
...
@@ -41,15 +39,20 @@ import org.apache.rocketmq.common.protocol.body.KVTable;
import
org.apache.rocketmq.common.protocol.body.RegisterBrokerBody
;
import
org.apache.rocketmq.common.protocol.body.SubscriptionGroupWrapper
;
import
org.apache.rocketmq.common.protocol.body.TopicConfigSerializeWrapper
;
import
org.apache.rocketmq.common.protocol.header.namesrv.GetRouteInfoRequestHeader
;
import
org.apache.rocketmq.common.protocol.header.namesrv.QueryDataVersionRequestHeader
;
import
org.apache.rocketmq.common.protocol.header.namesrv.QueryDataVersionResponseHeader
;
import
org.apache.rocketmq.common.protocol.header.namesrv.RegisterBrokerRequestHeader
;
import
org.apache.rocketmq.common.protocol.header.namesrv.RegisterBrokerResponseHeader
;
import
org.apache.rocketmq.common.protocol.header.namesrv.UnRegisterBrokerRequestHeader
;
import
org.apache.rocketmq.common.protocol.route.TopicRouteData
;
import
org.apache.rocketmq.logging.InternalLogger
;
import
org.apache.rocketmq.logging.InternalLoggerFactory
;
import
org.apache.rocketmq.remoting.RPCHook
;
import
org.apache.rocketmq.remoting.RemotingClient
;
import
org.apache.rocketmq.remoting.exception.RemotingCommandException
;
import
org.apache.rocketmq.remoting.exception.RemotingConnectException
;
import
org.apache.rocketmq.remoting.exception.RemotingException
;
import
org.apache.rocketmq.remoting.exception.RemotingSendRequestException
;
import
org.apache.rocketmq.remoting.exception.RemotingTimeoutException
;
import
org.apache.rocketmq.remoting.exception.RemotingTooMuchRequestException
;
...
...
@@ -394,4 +397,39 @@ public class BrokerOuterAPI {
public
void
registerRPCHook
(
RPCHook
rpcHook
)
{
remotingClient
.
registerRPCHook
(
rpcHook
);
}
public
TopicRouteData
getTopicRouteInfoFromNameServer
(
final
String
topic
,
final
long
timeoutMillis
)
throws
RemotingException
,
MQBrokerException
,
InterruptedException
{
return
getTopicRouteInfoFromNameServer
(
topic
,
timeoutMillis
,
true
);
}
public
TopicRouteData
getTopicRouteInfoFromNameServer
(
final
String
topic
,
final
long
timeoutMillis
,
boolean
allowTopicNotExist
)
throws
MQBrokerException
,
InterruptedException
,
RemotingTimeoutException
,
RemotingSendRequestException
,
RemotingConnectException
{
GetRouteInfoRequestHeader
requestHeader
=
new
GetRouteInfoRequestHeader
();
requestHeader
.
setTopic
(
topic
);
RemotingCommand
request
=
RemotingCommand
.
createRequestCommand
(
RequestCode
.
GET_ROUTEINFO_BY_TOPIC
,
requestHeader
);
RemotingCommand
response
=
this
.
remotingClient
.
invokeSync
(
null
,
request
,
timeoutMillis
);
assert
response
!=
null
;
switch
(
response
.
getCode
())
{
case
ResponseCode
.
TOPIC_NOT_EXIST
:
{
if
(
allowTopicNotExist
)
{
log
.
warn
(
"get Topic [{}] RouteInfoFromNameServer is not exist value"
,
topic
);
}
break
;
}
case
ResponseCode
.
SUCCESS
:
{
byte
[]
body
=
response
.
getBody
();
if
(
body
!=
null
)
{
return
TopicRouteData
.
decode
(
body
,
TopicRouteData
.
class
);
}
}
default
:
break
;
}
throw
new
MQBrokerException
(
response
.
getCode
(),
response
.
getRemark
());
}
}
broker/src/main/java/org/apache/rocketmq/broker/processor/AckMessageProcessor.java
0 → 100644
浏览文件 @
ced6b023
/*
* 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.processor
;
import
com.alibaba.fastjson.JSON
;
import
io.netty.channel.Channel
;
import
io.netty.channel.ChannelHandlerContext
;
import
org.apache.rocketmq.broker.BrokerController
;
import
org.apache.rocketmq.broker.util.MsgUtil
;
import
org.apache.rocketmq.common.KeyBuilder
;
import
org.apache.rocketmq.common.PopAckConstants
;
import
org.apache.rocketmq.common.TopicConfig
;
import
org.apache.rocketmq.common.constant.LoggerName
;
import
org.apache.rocketmq.common.help.FAQUrl
;
import
org.apache.rocketmq.common.message.MessageConst
;
import
org.apache.rocketmq.common.message.MessageDecoder
;
import
org.apache.rocketmq.common.protocol.ResponseCode
;
import
org.apache.rocketmq.common.protocol.header.AckMessageRequestHeader
;
import
org.apache.rocketmq.common.protocol.header.ExtraInfoUtil
;
import
org.apache.rocketmq.common.utils.DataConverter
;
import
org.apache.rocketmq.logging.InternalLogger
;
import
org.apache.rocketmq.logging.InternalLoggerFactory
;
import
org.apache.rocketmq.remoting.common.RemotingHelper
;
import
org.apache.rocketmq.remoting.exception.RemotingCommandException
;
import
org.apache.rocketmq.remoting.netty.NettyRequestProcessor
;
import
org.apache.rocketmq.remoting.protocol.RemotingCommand
;
import
org.apache.rocketmq.store.MessageExtBrokerInner
;
import
org.apache.rocketmq.store.PutMessageResult
;
import
org.apache.rocketmq.store.PutMessageStatus
;
import
org.apache.rocketmq.store.pop.AckMsg
;
public
class
AckMessageProcessor
implements
NettyRequestProcessor
{
private
static
final
InternalLogger
POP_LOGGER
=
InternalLoggerFactory
.
getLogger
(
LoggerName
.
ROCKETMQ_POP_LOGGER_NAME
);
private
final
BrokerController
brokerController
;
private
String
reviveTopic
;
private
PopReviveService
[]
popReviveServices
;
public
AckMessageProcessor
(
final
BrokerController
brokerController
)
{
this
.
brokerController
=
brokerController
;
this
.
reviveTopic
=
PopAckConstants
.
REVIVE_TOPIC
+
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
(
i
,
brokerController
,
reviveTopic
);
}
}
@Override
public
RemotingCommand
processRequest
(
final
ChannelHandlerContext
ctx
,
RemotingCommand
request
)
throws
RemotingCommandException
{
return
this
.
processRequest
(
ctx
.
channel
(),
request
,
true
);
}
@Override
public
boolean
rejectRequest
()
{
return
false
;
}
public
void
startPopReviveService
()
{
for
(
PopReviveService
popReviveService
:
popReviveServices
)
{
popReviveService
.
start
();
}
}
public
void
shutdownPopReviveService
()
{
for
(
PopReviveService
popReviveService
:
popReviveServices
)
{
popReviveService
.
shutdown
();
}
}
private
RemotingCommand
processRequest
(
final
Channel
channel
,
RemotingCommand
request
,
boolean
brokerAllowSuspend
)
throws
RemotingCommandException
{
final
AckMessageRequestHeader
requestHeader
=
(
AckMessageRequestHeader
)
request
.
decodeCommandCustomHeader
(
AckMessageRequestHeader
.
class
);
MessageExtBrokerInner
msgInner
=
new
MessageExtBrokerInner
();
AckMsg
ackMsg
=
new
AckMsg
();
RemotingCommand
response
=
RemotingCommand
.
createResponseCommand
(
ResponseCode
.
SUCCESS
,
null
);
response
.
setOpaque
(
request
.
getOpaque
());
TopicConfig
topicConfig
=
this
.
brokerController
.
getTopicConfigManager
().
selectTopicConfig
(
requestHeader
.
getTopic
());
if
(
null
==
topicConfig
)
{
POP_LOGGER
.
error
(
"The topic {} not exist, consumer: {} "
,
requestHeader
.
getTopic
(),
RemotingHelper
.
parseChannelRemoteAddr
(
channel
));
response
.
setCode
(
ResponseCode
.
TOPIC_NOT_EXIST
);
response
.
setRemark
(
String
.
format
(
"topic[%s] not exist, apply first please! %s"
,
requestHeader
.
getTopic
(),
FAQUrl
.
suggestTodo
(
FAQUrl
.
APPLY_TOPIC_URL
)));
return
response
;
}
if
(
requestHeader
.
getQueueId
()
>=
topicConfig
.
getReadQueueNums
()
||
requestHeader
.
getQueueId
()
<
0
)
{
String
errorInfo
=
String
.
format
(
"queueId[%d] is illegal, topic:[%s] topicConfig.readQueueNums:[%d] consumer:[%s]"
,
requestHeader
.
getQueueId
(),
requestHeader
.
getTopic
(),
topicConfig
.
getReadQueueNums
(),
channel
.
remoteAddress
());
POP_LOGGER
.
warn
(
errorInfo
);
response
.
setCode
(
ResponseCode
.
MESSAGE_ILLEGAL
);
response
.
setRemark
(
errorInfo
);
return
response
;
}
long
minOffset
=
this
.
brokerController
.
getMessageStore
().
getMinOffsetInQueue
(
requestHeader
.
getTopic
(),
requestHeader
.
getQueueId
());
long
maxOffset
=
this
.
brokerController
.
getMessageStore
().
getMaxOffsetInQueue
(
requestHeader
.
getTopic
(),
requestHeader
.
getQueueId
());
if
(
requestHeader
.
getOffset
()
<
minOffset
||
requestHeader
.
getOffset
()
>
maxOffset
)
{
response
.
setCode
(
ResponseCode
.
NO_MESSAGE
);
return
response
;
}
String
[]
extraInfo
=
ExtraInfoUtil
.
split
(
requestHeader
.
getExtraInfo
());
ackMsg
.
setAckOffset
(
requestHeader
.
getOffset
());
ackMsg
.
setStartOffset
(
ExtraInfoUtil
.
getCkQueueOffset
(
extraInfo
));
ackMsg
.
setConsumerGroup
(
requestHeader
.
getConsumerGroup
());
ackMsg
.
setTopic
(
requestHeader
.
getTopic
());
ackMsg
.
setQueueId
(
requestHeader
.
getQueueId
());
ackMsg
.
setPopTime
(
ExtraInfoUtil
.
getPopTime
(
extraInfo
));
int
rqId
=
ExtraInfoUtil
.
getReviveQid
(
extraInfo
);
if
(
rqId
==
KeyBuilder
.
POP_ORDER_REVIVE_QUEUE
)
{
// order
String
lockKey
=
requestHeader
.
getTopic
()
+
PopAckConstants
.
SPLIT
+
requestHeader
.
getConsumerGroup
()
+
PopAckConstants
.
SPLIT
+
requestHeader
.
getQueueId
();
long
oldOffset
=
this
.
brokerController
.
getConsumerOffsetManager
().
queryOffset
(
requestHeader
.
getConsumerGroup
(),
requestHeader
.
getTopic
(),
requestHeader
.
getQueueId
());
if
(
requestHeader
.
getOffset
()
<
oldOffset
)
{
return
response
;
}
while
(!
this
.
brokerController
.
getPopMessageProcessor
().
getQueueLockManager
().
tryLock
(
lockKey
))
{
}
try
{
oldOffset
=
this
.
brokerController
.
getConsumerOffsetManager
().
queryOffset
(
requestHeader
.
getConsumerGroup
(),
requestHeader
.
getTopic
(),
requestHeader
.
getQueueId
());
if
(
requestHeader
.
getOffset
()
<
oldOffset
)
{
return
response
;
}
long
nextOffset
=
brokerController
.
getConsumerOrderInfoManager
().
commitAndNext
(
requestHeader
.
getTopic
(),
requestHeader
.
getConsumerGroup
(),
requestHeader
.
getQueueId
(),
requestHeader
.
getOffset
());
if
(
nextOffset
>
-
1
)
{
this
.
brokerController
.
getConsumerOffsetManager
().
commitOffset
(
channel
.
remoteAddress
().
toString
(),
requestHeader
.
getConsumerGroup
(),
requestHeader
.
getTopic
(),
requestHeader
.
getQueueId
(),
nextOffset
);
this
.
brokerController
.
getPopMessageProcessor
().
notifyMessageArriving
(
requestHeader
.
getTopic
(),
requestHeader
.
getConsumerGroup
(),
requestHeader
.
getQueueId
());
}
else
if
(
nextOffset
==
-
1
)
{
String
errorInfo
=
String
.
format
(
"offset is illegal, key:%s, old:%d, commit:%d, next:%d, %s"
,
lockKey
,
oldOffset
,
requestHeader
.
getOffset
(),
nextOffset
,
channel
.
remoteAddress
());
POP_LOGGER
.
warn
(
errorInfo
);
response
.
setCode
(
ResponseCode
.
MESSAGE_ILLEGAL
);
response
.
setRemark
(
errorInfo
);
return
response
;
}
}
finally
{
this
.
brokerController
.
getPopMessageProcessor
().
getQueueLockManager
().
unLock
(
lockKey
);
}
return
response
;
}
if
(
this
.
brokerController
.
getPopMessageProcessor
().
getPopBufferMergeService
().
addAk
(
rqId
,
ackMsg
))
{
return
response
;
}
msgInner
.
setTopic
(
reviveTopic
);
msgInner
.
setBody
(
JSON
.
toJSONString
(
ackMsg
).
getBytes
(
DataConverter
.
charset
));
//msgInner.setQueueId(Integer.valueOf(extraInfo[3]));
msgInner
.
setQueueId
(
rqId
);
msgInner
.
setTags
(
PopAckConstants
.
ACK_TAG
);
msgInner
.
setBornTimestamp
(
System
.
currentTimeMillis
());
msgInner
.
setBornHost
(
this
.
brokerController
.
getStoreHost
());
msgInner
.
setStoreHost
(
this
.
brokerController
.
getStoreHost
());
MsgUtil
.
setMessageDeliverTime
(
this
.
brokerController
,
msgInner
,
ExtraInfoUtil
.
getPopTime
(
extraInfo
)
+
ExtraInfoUtil
.
getInvisibleTime
(
extraInfo
));
msgInner
.
getProperties
().
put
(
MessageConst
.
PROPERTY_UNIQ_CLIENT_MESSAGE_ID_KEYIDX
,
PopMessageProcessor
.
genAckUniqueId
(
ackMsg
));
msgInner
.
setPropertiesString
(
MessageDecoder
.
messageProperties2String
(
msgInner
.
getProperties
()));
PutMessageResult
putMessageResult
=
this
.
brokerController
.
getMessageStore
().
putMessage
(
msgInner
);
if
(
putMessageResult
.
getPutMessageStatus
()
!=
PutMessageStatus
.
PUT_OK
&&
putMessageResult
.
getPutMessageStatus
()
!=
PutMessageStatus
.
FLUSH_DISK_TIMEOUT
&&
putMessageResult
.
getPutMessageStatus
()
!=
PutMessageStatus
.
FLUSH_SLAVE_TIMEOUT
&&
putMessageResult
.
getPutMessageStatus
()
!=
PutMessageStatus
.
SLAVE_NOT_AVAILABLE
)
{
POP_LOGGER
.
error
(
"put ack msg error:"
+
putMessageResult
);
}
return
response
;
}
}
broker/src/main/java/org/apache/rocketmq/broker/processor/ChangeInvisibleTimeProcessor.java
0 → 100644
浏览文件 @
ced6b023
/*
* 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.processor
;
import
com.alibaba.fastjson.JSON
;
import
io.netty.channel.Channel
;
import
io.netty.channel.ChannelHandlerContext
;
import
org.apache.rocketmq.broker.BrokerController
;
import
org.apache.rocketmq.broker.util.MsgUtil
;
import
org.apache.rocketmq.common.PopAckConstants
;
import
org.apache.rocketmq.common.TopicConfig
;
import
org.apache.rocketmq.common.constant.LoggerName
;
import
org.apache.rocketmq.common.help.FAQUrl
;
import
org.apache.rocketmq.common.message.MessageConst
;
import
org.apache.rocketmq.common.message.MessageDecoder
;
import
org.apache.rocketmq.common.protocol.ResponseCode
;
import
org.apache.rocketmq.common.protocol.header.ChangeInvisibleTimeRequestHeader
;
import
org.apache.rocketmq.common.protocol.header.ChangeInvisibleTimeResponseHeader
;
import
org.apache.rocketmq.common.protocol.header.ExtraInfoUtil
;
import
org.apache.rocketmq.common.utils.DataConverter
;
import
org.apache.rocketmq.logging.InternalLogger
;
import
org.apache.rocketmq.logging.InternalLoggerFactory
;
import
org.apache.rocketmq.remoting.common.RemotingHelper
;
import
org.apache.rocketmq.remoting.exception.RemotingCommandException
;
import
org.apache.rocketmq.remoting.netty.NettyRequestProcessor
;
import
org.apache.rocketmq.remoting.protocol.RemotingCommand
;
import
org.apache.rocketmq.store.MessageExtBrokerInner
;
import
org.apache.rocketmq.store.PutMessageResult
;
import
org.apache.rocketmq.store.PutMessageStatus
;
import
org.apache.rocketmq.store.pop.AckMsg
;
import
org.apache.rocketmq.store.pop.PopCheckPoint
;
public
class
ChangeInvisibleTimeProcessor
implements
NettyRequestProcessor
{
private
static
final
InternalLogger
POP_LOGGER
=
InternalLoggerFactory
.
getLogger
(
LoggerName
.
ROCKETMQ_POP_LOGGER_NAME
);
private
final
BrokerController
brokerController
;
private
String
reviveTopic
;
public
ChangeInvisibleTimeProcessor
(
final
BrokerController
brokerController
)
{
this
.
brokerController
=
brokerController
;
this
.
reviveTopic
=
PopAckConstants
.
REVIVE_TOPIC
+
this
.
brokerController
.
getBrokerConfig
().
getBrokerClusterName
();
}
@Override
public
RemotingCommand
processRequest
(
final
ChannelHandlerContext
ctx
,
RemotingCommand
request
)
throws
RemotingCommandException
{
return
this
.
processRequest
(
ctx
.
channel
(),
request
,
true
);
}
@Override
public
boolean
rejectRequest
()
{
return
false
;
}
private
RemotingCommand
processRequest
(
final
Channel
channel
,
RemotingCommand
request
,
boolean
brokerAllowSuspend
)
throws
RemotingCommandException
{
final
ChangeInvisibleTimeRequestHeader
requestHeader
=
(
ChangeInvisibleTimeRequestHeader
)
request
.
decodeCommandCustomHeader
(
ChangeInvisibleTimeRequestHeader
.
class
);
RemotingCommand
response
=
RemotingCommand
.
createResponseCommand
(
ChangeInvisibleTimeResponseHeader
.
class
);
response
.
setCode
(
ResponseCode
.
SUCCESS
);
response
.
setOpaque
(
request
.
getOpaque
());
final
ChangeInvisibleTimeResponseHeader
responseHeader
=
(
ChangeInvisibleTimeResponseHeader
)
response
.
readCustomHeader
();
TopicConfig
topicConfig
=
this
.
brokerController
.
getTopicConfigManager
().
selectTopicConfig
(
requestHeader
.
getTopic
());
if
(
null
==
topicConfig
)
{
POP_LOGGER
.
error
(
"The topic {} not exist, consumer: {} "
,
requestHeader
.
getTopic
(),
RemotingHelper
.
parseChannelRemoteAddr
(
channel
));
response
.
setCode
(
ResponseCode
.
TOPIC_NOT_EXIST
);
response
.
setRemark
(
String
.
format
(
"topic[%s] not exist, apply first please! %s"
,
requestHeader
.
getTopic
(),
FAQUrl
.
suggestTodo
(
FAQUrl
.
APPLY_TOPIC_URL
)));
return
response
;
}
if
(
requestHeader
.
getQueueId
()
>=
topicConfig
.
getReadQueueNums
()
||
requestHeader
.
getQueueId
()
<
0
)
{
String
errorInfo
=
String
.
format
(
"queueId[%d] is illegal, topic:[%s] topicConfig.readQueueNums:[%d] consumer:[%s]"
,
requestHeader
.
getQueueId
(),
requestHeader
.
getTopic
(),
topicConfig
.
getReadQueueNums
(),
channel
.
remoteAddress
());
POP_LOGGER
.
warn
(
errorInfo
);
response
.
setCode
(
ResponseCode
.
MESSAGE_ILLEGAL
);
response
.
setRemark
(
errorInfo
);
return
response
;
}
long
minOffset
=
this
.
brokerController
.
getMessageStore
().
getMinOffsetInQueue
(
requestHeader
.
getTopic
(),
requestHeader
.
getQueueId
());
long
maxOffset
=
this
.
brokerController
.
getMessageStore
().
getMaxOffsetInQueue
(
requestHeader
.
getTopic
(),
requestHeader
.
getQueueId
());
if
(
requestHeader
.
getOffset
()
<
minOffset
||
requestHeader
.
getOffset
()
>
maxOffset
)
{
response
.
setCode
(
ResponseCode
.
NO_MESSAGE
);
return
response
;
}
String
[]
extraInfo
=
ExtraInfoUtil
.
split
(
requestHeader
.
getExtraInfo
());
// add new ck
long
now
=
System
.
currentTimeMillis
();
PutMessageResult
ckResult
=
appendCheckPoint
(
requestHeader
,
ExtraInfoUtil
.
getReviveQid
(
extraInfo
),
requestHeader
.
getQueueId
(),
requestHeader
.
getOffset
(),
now
);
if
(
ckResult
.
getPutMessageStatus
()
!=
PutMessageStatus
.
PUT_OK
&&
ckResult
.
getPutMessageStatus
()
!=
PutMessageStatus
.
FLUSH_DISK_TIMEOUT
&&
ckResult
.
getPutMessageStatus
()
!=
PutMessageStatus
.
FLUSH_SLAVE_TIMEOUT
&&
ckResult
.
getPutMessageStatus
()
!=
PutMessageStatus
.
SLAVE_NOT_AVAILABLE
)
{
POP_LOGGER
.
error
(
"change Invisible, put new ck error: {}"
,
ckResult
);
response
.
setCode
(
ResponseCode
.
SYSTEM_ERROR
);
return
response
;
}
// ack old msg.
try
{
ackOrigin
(
requestHeader
,
extraInfo
);
}
catch
(
Throwable
e
)
{
POP_LOGGER
.
error
(
"change Invisible, put ack msg error: {}, {}"
,
requestHeader
.
getExtraInfo
(),
e
.
getMessage
());
// cancel new ck?
}
responseHeader
.
setInvisibleTime
(
requestHeader
.
getInvisibleTime
());
responseHeader
.
setPopTime
(
now
);
responseHeader
.
setReviveQid
(
ExtraInfoUtil
.
getReviveQid
(
extraInfo
));
return
response
;
}
private
void
ackOrigin
(
final
ChangeInvisibleTimeRequestHeader
requestHeader
,
String
[]
extraInfo
)
{
MessageExtBrokerInner
msgInner
=
new
MessageExtBrokerInner
();
AckMsg
ackMsg
=
new
AckMsg
();
ackMsg
.
setAckOffset
(
requestHeader
.
getOffset
());
ackMsg
.
setStartOffset
(
ExtraInfoUtil
.
getCkQueueOffset
(
extraInfo
));
ackMsg
.
setConsumerGroup
(
requestHeader
.
getConsumerGroup
());
ackMsg
.
setTopic
(
requestHeader
.
getTopic
());
ackMsg
.
setQueueId
(
requestHeader
.
getQueueId
());
ackMsg
.
setPopTime
(
ExtraInfoUtil
.
getPopTime
(
extraInfo
));
int
rqId
=
ExtraInfoUtil
.
getReviveQid
(
extraInfo
);
if
(
brokerController
.
getPopMessageProcessor
().
getPopBufferMergeService
().
addAk
(
rqId
,
ackMsg
))
{
return
;
}
msgInner
.
setTopic
(
reviveTopic
);
msgInner
.
setBody
(
JSON
.
toJSONString
(
ackMsg
).
getBytes
(
DataConverter
.
charset
));
msgInner
.
setQueueId
(
rqId
);
msgInner
.
setTags
(
PopAckConstants
.
ACK_TAG
);
msgInner
.
setBornTimestamp
(
System
.
currentTimeMillis
());
msgInner
.
setBornHost
(
this
.
brokerController
.
getStoreHost
());
msgInner
.
setStoreHost
(
this
.
brokerController
.
getStoreHost
());
MsgUtil
.
setMessageDeliverTime
(
this
.
brokerController
,
msgInner
,
ExtraInfoUtil
.
getPopTime
(
extraInfo
)
+
ExtraInfoUtil
.
getInvisibleTime
(
extraInfo
));
msgInner
.
getProperties
().
put
(
MessageConst
.
PROPERTY_UNIQ_CLIENT_MESSAGE_ID_KEYIDX
,
PopMessageProcessor
.
genAckUniqueId
(
ackMsg
));
msgInner
.
setPropertiesString
(
MessageDecoder
.
messageProperties2String
(
msgInner
.
getProperties
()));
PutMessageResult
putMessageResult
=
this
.
brokerController
.
getMessageStore
().
putMessage
(
msgInner
);
if
(
putMessageResult
.
getPutMessageStatus
()
!=
PutMessageStatus
.
PUT_OK
&&
putMessageResult
.
getPutMessageStatus
()
!=
PutMessageStatus
.
FLUSH_DISK_TIMEOUT
&&
putMessageResult
.
getPutMessageStatus
()
!=
PutMessageStatus
.
FLUSH_SLAVE_TIMEOUT
&&
putMessageResult
.
getPutMessageStatus
()
!=
PutMessageStatus
.
SLAVE_NOT_AVAILABLE
)
{
POP_LOGGER
.
error
(
"change Invisible, put ack msg fail: {}, {}"
,
ackMsg
,
putMessageResult
);
}
}
private
PutMessageResult
appendCheckPoint
(
final
ChangeInvisibleTimeRequestHeader
requestHeader
,
int
reviveQid
,
int
queueId
,
long
offset
,
long
popTime
)
{
// add check point msg to revive log
MessageExtBrokerInner
msgInner
=
new
MessageExtBrokerInner
();
msgInner
.
setTopic
(
reviveTopic
);
PopCheckPoint
ck
=
new
PopCheckPoint
();
ck
.
setBitMap
(
0
);
ck
.
setNum
((
byte
)
1
);
ck
.
setPopTime
(
popTime
);
ck
.
setInvisibleTime
(
requestHeader
.
getInvisibleTime
());
ck
.
getStartOffset
(
offset
);
ck
.
setCId
(
requestHeader
.
getConsumerGroup
());
ck
.
setTopic
(
requestHeader
.
getTopic
());
ck
.
setQueueId
((
byte
)
queueId
);
ck
.
addDiff
(
0
);
msgInner
.
setBody
(
JSON
.
toJSONString
(
ck
).
getBytes
(
DataConverter
.
charset
));
msgInner
.
setQueueId
(
reviveQid
);
msgInner
.
setTags
(
PopAckConstants
.
CK_TAG
);
msgInner
.
setBornTimestamp
(
System
.
currentTimeMillis
());
msgInner
.
setBornHost
(
this
.
brokerController
.
getStoreHost
());
msgInner
.
setStoreHost
(
this
.
brokerController
.
getStoreHost
());
MsgUtil
.
setMessageDeliverTime
(
this
.
brokerController
,
msgInner
,
ck
.
getReviveTime
()
-
PopAckConstants
.
ackTimeInterval
);
msgInner
.
getProperties
().
put
(
MessageConst
.
PROPERTY_UNIQ_CLIENT_MESSAGE_ID_KEYIDX
,
PopMessageProcessor
.
genCkUniqueId
(
ck
));
msgInner
.
setPropertiesString
(
MessageDecoder
.
messageProperties2String
(
msgInner
.
getProperties
()));
PutMessageResult
putMessageResult
=
this
.
brokerController
.
getMessageStore
().
putMessage
(
msgInner
);
if
(
brokerController
.
getBrokerConfig
().
isEnablePopLog
())
{
POP_LOGGER
.
info
(
"change Invisible , appendCheckPoint, topic {}, queueId {},reviveId {}, cid {}, startOffset {}, rt {}, result {}"
,
requestHeader
.
getTopic
(),
queueId
,
reviveQid
,
requestHeader
.
getConsumerGroup
(),
offset
,
ck
.
getReviveTime
(),
putMessageResult
);
}
return
putMessageResult
;
}
}
broker/src/main/java/org/apache/rocketmq/broker/processor/PopBufferMergeService.java
0 → 100644
浏览文件 @
ced6b023
/*
* 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.processor
;
import
com.alibaba.fastjson.JSON
;
import
java.util.Iterator
;
import
java.util.Map
;
import
java.util.concurrent.ConcurrentHashMap
;
import
java.util.concurrent.LinkedBlockingDeque
;
import
java.util.concurrent.atomic.AtomicInteger
;
import
org.apache.rocketmq.broker.BrokerController
;
import
org.apache.rocketmq.broker.util.MsgUtil
;
import
org.apache.rocketmq.common.KeyBuilder
;
import
org.apache.rocketmq.common.PopAckConstants
;
import
org.apache.rocketmq.common.ServiceThread
;
import
org.apache.rocketmq.common.constant.LoggerName
;
import
org.apache.rocketmq.common.message.MessageConst
;
import
org.apache.rocketmq.common.message.MessageDecoder
;
import
org.apache.rocketmq.common.utils.DataConverter
;
import
org.apache.rocketmq.logging.InternalLogger
;
import
org.apache.rocketmq.logging.InternalLoggerFactory
;
import
org.apache.rocketmq.store.MessageExtBrokerInner
;
import
org.apache.rocketmq.store.PutMessageResult
;
import
org.apache.rocketmq.store.PutMessageStatus
;
import
org.apache.rocketmq.store.config.BrokerRole
;
import
org.apache.rocketmq.store.pop.AckMsg
;
import
org.apache.rocketmq.store.pop.PopCheckPoint
;
public
class
PopBufferMergeService
extends
ServiceThread
{
private
static
final
InternalLogger
POP_LOGGER
=
InternalLoggerFactory
.
getLogger
(
LoggerName
.
ROCKETMQ_POP_LOGGER_NAME
);
ConcurrentHashMap
<
String
/*mergeKey*/
,
PopCheckPointWrapper
>
buffer
=
new
ConcurrentHashMap
<>(
1024
*
16
);
ConcurrentHashMap
<
String
/*topic@cid@queueId*/
,
QueueWithTime
<
PopCheckPointWrapper
>>
commitOffsets
=
new
ConcurrentHashMap
<>();
private
volatile
boolean
serving
=
true
;
private
AtomicInteger
counter
=
new
AtomicInteger
(
0
);
private
int
scanTimes
=
0
;
private
final
BrokerController
brokerController
;
private
final
PopMessageProcessor
popMessageProcessor
;
private
final
PopMessageProcessor
.
QueueLockManager
queueLockManager
;
private
final
long
interval
=
5
;
private
final
long
minute5
=
5
*
60
*
1000
;
private
final
int
countOfMinute1
=
(
int
)
(
60
*
1000
/
interval
);
private
final
int
countOfSecond1
=
(
int
)
(
1000
/
interval
);
private
final
int
countOfSecond30
=
(
int
)
(
30
*
1000
/
interval
);
private
volatile
boolean
master
=
false
;
public
PopBufferMergeService
(
BrokerController
brokerController
,
PopMessageProcessor
popMessageProcessor
)
{
super
();
this
.
brokerController
=
brokerController
;
this
.
popMessageProcessor
=
popMessageProcessor
;
this
.
queueLockManager
=
popMessageProcessor
.
getQueueLockManager
();
}
private
boolean
checkAndSetMaster
()
{
this
.
master
=
brokerController
.
getMessageStoreConfig
().
getBrokerRole
()
!=
BrokerRole
.
SLAVE
;
return
this
.
master
;
}
@Override
public
String
getServiceName
()
{
return
"PopBufferMergeService"
;
}
@Override
public
void
run
()
{
// scan
while
(!
this
.
isStopped
())
{
try
{
if
(!
checkAndSetMaster
())
{
// slave
this
.
waitForRunning
(
interval
*
200
*
5
);
POP_LOGGER
.
info
(
"Broker is {}, {}, clear all data"
,
brokerController
.
getMessageStoreConfig
().
getBrokerRole
(),
this
.
master
);
this
.
buffer
.
clear
();
this
.
commitOffsets
.
clear
();
continue
;
}
scan
();
if
(
scanTimes
%
countOfSecond30
==
0
)
{
scanGarbage
();
}
this
.
waitForRunning
(
interval
);
if
(!
this
.
serving
&&
this
.
buffer
.
size
()
==
0
&&
totalSize
()
==
0
)
{
this
.
serving
=
true
;
}
}
catch
(
Throwable
e
)
{
POP_LOGGER
.
error
(
"PopBufferMergeService error"
,
e
);
this
.
waitForRunning
(
3000
);
}
}
this
.
serving
=
false
;
try
{
Thread
.
sleep
(
2000
);
}
catch
(
InterruptedException
e
)
{
}
if
(!
checkAndSetMaster
())
{
return
;
}
while
(
this
.
buffer
.
size
()
>
0
||
totalSize
()
>
0
)
{
scan
();
}
}
private
int
scanCommitOffset
()
{
Iterator
<
Map
.
Entry
<
String
,
QueueWithTime
<
PopCheckPointWrapper
>>>
iterator
=
this
.
commitOffsets
.
entrySet
().
iterator
();
int
count
=
0
;
while
(
iterator
.
hasNext
())
{
Map
.
Entry
<
String
,
QueueWithTime
<
PopCheckPointWrapper
>>
entry
=
iterator
.
next
();
LinkedBlockingDeque
<
PopCheckPointWrapper
>
queue
=
entry
.
getValue
().
get
();
PopCheckPointWrapper
pointWrapper
;
while
((
pointWrapper
=
queue
.
peek
())
!=
null
)
{
// 1. just offset & stored, not processed by scan
// 2. ck is buffer(acked)
// 3. ck is buffer(not all acked), all ak are stored and ck is stored
if
((
pointWrapper
.
isJustOffset
()
&&
pointWrapper
.
isCkStored
())
||
isCkDone
(
pointWrapper
)
||
(
isCkDoneForFinish
(
pointWrapper
)
&&
pointWrapper
.
isCkStored
()))
{
if
(
commitOffset
(
pointWrapper
))
{
queue
.
poll
();
}
else
{
break
;
}
}
else
{
if
(
System
.
currentTimeMillis
()
-
pointWrapper
.
getCk
().
getPopTime
()
>
brokerController
.
getBrokerConfig
().
getPopCkStayBufferTime
()
*
2
)
{
POP_LOGGER
.
warn
(
"[PopBuffer] ck offset long time not commit, {}"
,
pointWrapper
);
}
break
;
}
}
final
int
qs
=
queue
.
size
();
count
+=
qs
;
if
(
qs
>
5000
&&
scanTimes
%
countOfSecond1
==
0
)
{
POP_LOGGER
.
info
(
"[PopBuffer] offset queue size too long, {}, {}"
,
entry
.
getKey
(),
qs
);
}
}
return
count
;
}
public
long
getLatestOffset
(
String
lockKey
)
{
QueueWithTime
<
PopCheckPointWrapper
>
queue
=
this
.
commitOffsets
.
get
(
lockKey
);
if
(
queue
==
null
)
{
return
-
1
;
}
PopCheckPointWrapper
pointWrapper
=
queue
.
get
().
peekLast
();
if
(
pointWrapper
!=
null
)
{
return
pointWrapper
.
getNextBeginOffset
();
}
return
-
1
;
}
public
long
getLatestOffset
(
String
topic
,
String
group
,
int
queueId
)
{
return
getLatestOffset
(
KeyBuilder
.
buildPollingKey
(
topic
,
group
,
queueId
));
}
private
void
scanGarbage
()
{
Iterator
<
Map
.
Entry
<
String
,
QueueWithTime
<
PopCheckPointWrapper
>>>
iterator
=
commitOffsets
.
entrySet
().
iterator
();
while
(
iterator
.
hasNext
())
{
Map
.
Entry
<
String
,
QueueWithTime
<
PopCheckPointWrapper
>>
entry
=
iterator
.
next
();
if
(
entry
.
getKey
()
==
null
)
{
continue
;
}
String
[]
keyArray
=
entry
.
getKey
().
split
(
PopAckConstants
.
SPLIT
);
if
(
keyArray
==
null
||
keyArray
.
length
!=
3
)
{
continue
;
}
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
);
iterator
.
remove
();
continue
;
}
if
(!
brokerController
.
getSubscriptionGroupManager
().
getSubscriptionGroupTable
().
containsKey
(
cid
))
{
POP_LOGGER
.
info
(
"[PopBuffer]remove not exit sub {} of topic {} in buffer!"
,
cid
,
topic
);
iterator
.
remove
();
continue
;
}
if
(
System
.
currentTimeMillis
()
-
entry
.
getValue
().
getTime
()
>
minute5
)
{
POP_LOGGER
.
info
(
"[PopBuffer]remove long time not used sub {} of topic {} in buffer!"
,
cid
,
topic
);
iterator
.
remove
();
continue
;
}
}
}
private
void
scan
()
{
long
startTime
=
System
.
currentTimeMillis
();
int
count
=
0
,
countCk
=
0
;
Iterator
<
Map
.
Entry
<
String
,
PopCheckPointWrapper
>>
iterator
=
buffer
.
entrySet
().
iterator
();
while
(
iterator
.
hasNext
())
{
Map
.
Entry
<
String
,
PopCheckPointWrapper
>
entry
=
iterator
.
next
();
PopCheckPointWrapper
pointWrapper
=
entry
.
getValue
();
// just process offset(already stored at pull thread), or buffer ck(not stored and ack finish)
if
((
pointWrapper
.
isJustOffset
()
&&
pointWrapper
.
isCkStored
())
||
isCkDone
(
pointWrapper
)
||
(
isCkDoneForFinish
(
pointWrapper
)
&&
pointWrapper
.
isCkStored
()))
{
if
(
brokerController
.
getBrokerConfig
().
isEnablePopLog
())
{
POP_LOGGER
.
info
(
"[PopBuffer]ck done, {}"
,
pointWrapper
);
}
iterator
.
remove
();
counter
.
decrementAndGet
();
continue
;
}
PopCheckPoint
point
=
pointWrapper
.
getCk
();
long
now
=
System
.
currentTimeMillis
();
boolean
removeCk
=
!
this
.
serving
;
// ck will be timeout
if
(
point
.
getReviveTime
()
-
now
<
brokerController
.
getBrokerConfig
().
getPopCkStayBufferTimeOut
())
{
removeCk
=
true
;
}
// the time stayed is too long
if
(
now
-
point
.
getPopTime
()
>
brokerController
.
getBrokerConfig
().
getPopCkStayBufferTime
())
{
removeCk
=
true
;
}
if
(
now
-
point
.
getPopTime
()
>
brokerController
.
getBrokerConfig
().
getPopCkStayBufferTime
()
*
2L
)
{
POP_LOGGER
.
warn
(
"[PopBuffer]ck finish fail, stay too long, {}"
,
pointWrapper
);
}
// double check
if
(
isCkDone
(
pointWrapper
))
{
continue
;
}
else
if
(
pointWrapper
.
isJustOffset
())
{
// just offset should be in store.
if
(
pointWrapper
.
getReviveQueueOffset
()
<
0
)
{
putCkToStore
(
pointWrapper
,
false
);
countCk
++;
}
continue
;
}
else
if
(
removeCk
)
{
// put buffer ak to store
if
(
pointWrapper
.
getReviveQueueOffset
()
<
0
)
{
putCkToStore
(
pointWrapper
,
false
);
countCk
++;
}
if
(!
pointWrapper
.
isCkStored
())
{
continue
;
}
for
(
byte
i
=
0
;
i
<
point
.
getNum
();
i
++)
{
// reput buffer ak to store
if
(
DataConverter
.
getBit
(
pointWrapper
.
getBits
().
get
(),
i
)
&&
!
DataConverter
.
getBit
(
pointWrapper
.
getToStoreBits
().
get
(),
i
))
{
if
(
putAckToStore
(
pointWrapper
,
i
))
{
count
++;
markBitCAS
(
pointWrapper
.
getToStoreBits
(),
i
);
}
}
}
if
(
isCkDoneForFinish
(
pointWrapper
)
&&
pointWrapper
.
isCkStored
())
{
if
(
brokerController
.
getBrokerConfig
().
isEnablePopLog
())
{
POP_LOGGER
.
info
(
"[PopBuffer]ck finish, {}"
,
pointWrapper
);
}
iterator
.
remove
();
counter
.
decrementAndGet
();
continue
;
}
}
}
int
offsetBufferSize
=
scanCommitOffset
();
long
eclipse
=
System
.
currentTimeMillis
()
-
startTime
;
if
(
eclipse
>
brokerController
.
getBrokerConfig
().
getPopCkStayBufferTimeOut
()
-
1000
)
{
POP_LOGGER
.
warn
(
"[PopBuffer]scan stop, because eclipse too long, PopBufferEclipse={}, "
+
"PopBufferToStoreAck={}, PopBufferToStoreCk={}, PopBufferSize={}, PopBufferOffsetSize={}"
,
eclipse
,
count
,
countCk
,
counter
.
get
(),
offsetBufferSize
);
this
.
serving
=
false
;
}
else
{
if
(
scanTimes
%
countOfSecond1
==
0
)
{
POP_LOGGER
.
info
(
"[PopBuffer]scan, PopBufferEclipse={}, "
+
"PopBufferToStoreAck={}, PopBufferToStoreCk={}, PopBufferSize={}, PopBufferOffsetSize={}"
,
eclipse
,
count
,
countCk
,
counter
.
get
(),
offsetBufferSize
);
}
}
scanTimes
++;
if
(
scanTimes
>=
countOfMinute1
)
{
counter
.
set
(
this
.
buffer
.
size
());
scanTimes
=
0
;
}
}
private
int
totalSize
()
{
int
count
=
0
;
Iterator
<
Map
.
Entry
<
String
,
QueueWithTime
<
PopCheckPointWrapper
>>>
iterator
=
this
.
commitOffsets
.
entrySet
().
iterator
();
while
(
iterator
.
hasNext
())
{
Map
.
Entry
<
String
,
QueueWithTime
<
PopCheckPointWrapper
>>
entry
=
iterator
.
next
();
LinkedBlockingDeque
<
PopCheckPointWrapper
>
queue
=
entry
.
getValue
().
get
();
count
+=
queue
.
size
();
}
return
count
;
}
private
void
markBitCAS
(
AtomicInteger
setBits
,
int
index
)
{
while
(
true
)
{
int
bits
=
setBits
.
get
();
if
(
DataConverter
.
getBit
(
bits
,
index
))
{
break
;
}
int
newBits
=
DataConverter
.
setBit
(
bits
,
index
,
true
);
if
(
setBits
.
compareAndSet
(
bits
,
newBits
))
{
break
;
}
}
}
private
boolean
commitOffset
(
final
PopCheckPointWrapper
wrapper
)
{
if
(
wrapper
.
getNextBeginOffset
()
<
0
)
{
return
true
;
}
final
PopCheckPoint
popCheckPoint
=
wrapper
.
getCk
();
final
String
lockKey
=
wrapper
.
getLockKey
();
if
(!
queueLockManager
.
tryLock
(
lockKey
))
{
return
false
;
}
try
{
final
long
offset
=
brokerController
.
getConsumerOffsetManager
().
queryOffset
(
popCheckPoint
.
getCId
(),
popCheckPoint
.
getTopic
(),
popCheckPoint
.
getQueueId
());
if
(
wrapper
.
getNextBeginOffset
()
>
offset
)
{
if
(
brokerController
.
getBrokerConfig
().
isEnablePopLog
())
{
POP_LOGGER
.
info
(
"Commit offset, {}, {}"
,
wrapper
,
offset
);
}
}
else
{
// maybe store offset is not correct.
POP_LOGGER
.
warn
(
"Commit offset, consumer offset less than store, {}, {}"
,
wrapper
,
offset
);
}
brokerController
.
getConsumerOffsetManager
().
commitOffset
(
getServiceName
(),
popCheckPoint
.
getCId
(),
popCheckPoint
.
getTopic
(),
popCheckPoint
.
getQueueId
(),
wrapper
.
getNextBeginOffset
());
}
finally
{
queueLockManager
.
unLock
(
lockKey
);
}
return
true
;
}
private
boolean
putOffsetQueue
(
PopCheckPointWrapper
pointWrapper
)
{
QueueWithTime
<
PopCheckPointWrapper
>
queue
=
this
.
commitOffsets
.
get
(
pointWrapper
.
getLockKey
());
if
(
queue
==
null
)
{
queue
=
new
QueueWithTime
<>();
QueueWithTime
old
=
this
.
commitOffsets
.
putIfAbsent
(
pointWrapper
.
getLockKey
(),
queue
);
if
(
old
!=
null
)
{
queue
=
old
;
}
}
queue
.
setTime
(
pointWrapper
.
getCk
().
getPopTime
());
return
queue
.
get
().
offer
(
pointWrapper
);
}
private
boolean
checkQueueOk
(
PopCheckPointWrapper
pointWrapper
)
{
QueueWithTime
<
PopCheckPointWrapper
>
queue
=
this
.
commitOffsets
.
get
(
pointWrapper
.
getLockKey
());
if
(
queue
==
null
)
{
return
true
;
}
return
queue
.
get
().
size
()
<
brokerController
.
getBrokerConfig
().
getPopCkOffsetMaxQueueSize
();
}
/**
* put to store && add to buffer.
* @param point
* @param reviveQueueId
* @param reviveQueueOffset
* @param nextBeginOffset
* @return
*/
public
void
addCkJustOffset
(
PopCheckPoint
point
,
int
reviveQueueId
,
long
reviveQueueOffset
,
long
nextBeginOffset
)
{
PopCheckPointWrapper
pointWrapper
=
new
PopCheckPointWrapper
(
reviveQueueId
,
reviveQueueOffset
,
point
,
nextBeginOffset
,
true
);
this
.
putCkToStore
(
pointWrapper
,
!
checkQueueOk
(
pointWrapper
));
putOffsetQueue
(
pointWrapper
);
this
.
buffer
.
put
(
pointWrapper
.
getMergeKey
(),
pointWrapper
);
this
.
counter
.
incrementAndGet
();
if
(
brokerController
.
getBrokerConfig
().
isEnablePopLog
())
{
POP_LOGGER
.
info
(
"[PopBuffer]add ck just offset, {}"
,
pointWrapper
);
}
}
public
void
addCkMock
(
String
group
,
String
topic
,
int
queueId
,
long
startOffset
,
long
invisibleTime
,
long
popTime
,
int
reviveQueueId
,
long
nextBeginOffset
)
{
final
PopCheckPoint
ck
=
new
PopCheckPoint
();
ck
.
setBitMap
(
0
);
ck
.
setNum
((
byte
)
0
);
ck
.
setPopTime
(
popTime
);
ck
.
setInvisibleTime
(
invisibleTime
);
ck
.
getStartOffset
(
startOffset
);
ck
.
setCId
(
group
);
ck
.
setTopic
(
topic
);
ck
.
setQueueId
((
byte
)
queueId
);
PopCheckPointWrapper
pointWrapper
=
new
PopCheckPointWrapper
(
reviveQueueId
,
Long
.
MAX_VALUE
,
ck
,
nextBeginOffset
,
true
);
pointWrapper
.
setCkStored
(
true
);
putOffsetQueue
(
pointWrapper
);
if
(
brokerController
.
getBrokerConfig
().
isEnablePopLog
())
{
POP_LOGGER
.
info
(
"[PopBuffer]add ck just offset, mocked, {}"
,
pointWrapper
);
}
}
public
boolean
addCk
(
PopCheckPoint
point
,
int
reviveQueueId
,
long
reviveQueueOffset
,
long
nextBeginOffset
)
{
// key: point.getT() + point.getC() + point.getQ() + point.getSo() + point.getPt()
if
(!
brokerController
.
getBrokerConfig
().
isEnablePopBufferMerge
())
{
return
false
;
}
if
(!
serving
)
{
return
false
;
}
long
now
=
System
.
currentTimeMillis
();
if
(
point
.
getReviveTime
()
-
now
<
brokerController
.
getBrokerConfig
().
getPopCkStayBufferTimeOut
()
+
1500
)
{
if
(
brokerController
.
getBrokerConfig
().
isEnablePopLog
())
{
POP_LOGGER
.
warn
(
"[PopBuffer]add ck, timeout, {}, {}"
,
point
,
now
);
}
return
false
;
}
if
(
this
.
counter
.
get
()
>
brokerController
.
getBrokerConfig
().
getPopCkMaxBufferSize
())
{
POP_LOGGER
.
warn
(
"[PopBuffer]add ck, max size, {}, {}"
,
point
,
this
.
counter
.
get
());
return
false
;
}
PopCheckPointWrapper
pointWrapper
=
new
PopCheckPointWrapper
(
reviveQueueId
,
reviveQueueOffset
,
point
,
nextBeginOffset
);
if
(!
checkQueueOk
(
pointWrapper
))
{
return
false
;
}
putOffsetQueue
(
pointWrapper
);
this
.
buffer
.
put
(
pointWrapper
.
getMergeKey
(),
pointWrapper
);
this
.
counter
.
incrementAndGet
();
if
(
brokerController
.
getBrokerConfig
().
isEnablePopLog
())
{
POP_LOGGER
.
info
(
"[PopBuffer]add ck, {}"
,
pointWrapper
);
}
return
true
;
}
public
boolean
addAk
(
int
reviveQid
,
AckMsg
ackMsg
)
{
if
(!
brokerController
.
getBrokerConfig
().
isEnablePopBufferMerge
())
{
return
false
;
}
if
(!
serving
)
{
return
false
;
}
try
{
PopCheckPointWrapper
pointWrapper
=
this
.
buffer
.
get
(
ackMsg
.
getTopic
()
+
ackMsg
.
getConsumerGroup
()
+
ackMsg
.
getQueueId
()
+
ackMsg
.
getStartOffset
()
+
ackMsg
.
getPopTime
());
if
(
pointWrapper
==
null
)
{
if
(
brokerController
.
getBrokerConfig
().
isEnablePopLog
())
{
POP_LOGGER
.
warn
(
"[PopBuffer]add ack fail, rqId={}, no ck, {}"
,
reviveQid
,
ackMsg
);
}
return
false
;
}
if
(
pointWrapper
.
isJustOffset
())
{
return
false
;
}
PopCheckPoint
point
=
pointWrapper
.
getCk
();
long
now
=
System
.
currentTimeMillis
();
if
(
point
.
getReviveTime
()
-
now
<
brokerController
.
getBrokerConfig
().
getPopCkStayBufferTimeOut
()
+
1500
)
{
if
(
brokerController
.
getBrokerConfig
().
isEnablePopLog
())
{
POP_LOGGER
.
warn
(
"[PopBuffer]add ack fail, rqId={}, almost timeout for revive, {}, {}, {}"
,
reviveQid
,
pointWrapper
,
ackMsg
,
now
);
}
return
false
;
}
if
(
now
-
point
.
getPopTime
()
>
brokerController
.
getBrokerConfig
().
getPopCkStayBufferTime
()
-
1500
)
{
if
(
brokerController
.
getBrokerConfig
().
isEnablePopLog
())
{
POP_LOGGER
.
warn
(
"[PopBuffer]add ack fail, rqId={}, stay too long, {}, {}, {}"
,
reviveQid
,
pointWrapper
,
ackMsg
,
now
);
}
return
false
;
}
int
indexOfAck
=
point
.
indexOfAck
(
ackMsg
.
getAckOffset
());
if
(
indexOfAck
>
-
1
)
{
markBitCAS
(
pointWrapper
.
getBits
(),
indexOfAck
);
}
else
{
POP_LOGGER
.
error
(
"[PopBuffer]Invalid index of ack, reviveQid={}, {}, {}"
,
reviveQid
,
ackMsg
,
point
);
return
true
;
}
if
(
brokerController
.
getBrokerConfig
().
isEnablePopLog
())
{
POP_LOGGER
.
info
(
"[PopBuffer]add ack, rqId={}, {}, {}"
,
reviveQid
,
pointWrapper
,
ackMsg
);
}
// // check ak done
// if (isCkDone(pointWrapper)) {
// // cancel ck for timer
// cancelCkTimer(pointWrapper);
// }
return
true
;
}
catch
(
Throwable
e
)
{
POP_LOGGER
.
error
(
"[PopBuffer]add ack error, rqId="
+
reviveQid
+
", "
+
ackMsg
,
e
);
}
return
false
;
}
private
void
putCkToStore
(
final
PopCheckPointWrapper
pointWrapper
,
final
boolean
runInCurrent
)
{
if
(
pointWrapper
.
getReviveQueueOffset
()
>=
0
)
{
return
;
}
MessageExtBrokerInner
msgInner
=
popMessageProcessor
.
buildCkMsg
(
pointWrapper
.
getCk
(),
pointWrapper
.
getReviveQueueId
());
PutMessageResult
putMessageResult
=
brokerController
.
getMessageStore
().
putMessage
(
msgInner
);
if
(
putMessageResult
.
getPutMessageStatus
()
!=
PutMessageStatus
.
PUT_OK
&&
putMessageResult
.
getPutMessageStatus
()
!=
PutMessageStatus
.
FLUSH_DISK_TIMEOUT
&&
putMessageResult
.
getPutMessageStatus
()
!=
PutMessageStatus
.
FLUSH_SLAVE_TIMEOUT
&&
putMessageResult
.
getPutMessageStatus
()
!=
PutMessageStatus
.
SLAVE_NOT_AVAILABLE
)
{
POP_LOGGER
.
error
(
"[PopBuffer]put ck to store fail: {}, {}"
,
pointWrapper
,
putMessageResult
);
return
;
}
pointWrapper
.
setCkStored
(
true
);
pointWrapper
.
setReviveQueueOffset
(
putMessageResult
.
getAppendMessageResult
().
getLogicsOffset
());
if
(
brokerController
.
getBrokerConfig
().
isEnablePopLog
())
{
POP_LOGGER
.
info
(
"[PopBuffer]put ck to store ok: {}, {}"
,
pointWrapper
,
putMessageResult
);
}
}
private
boolean
putAckToStore
(
final
PopCheckPointWrapper
pointWrapper
,
byte
msgIndex
)
{
PopCheckPoint
point
=
pointWrapper
.
getCk
();
MessageExtBrokerInner
msgInner
=
new
MessageExtBrokerInner
();
final
AckMsg
ackMsg
=
new
AckMsg
();
ackMsg
.
setAckOffset
(
point
.
ackOffsetByIndex
(
msgIndex
));
ackMsg
.
setStartOffset
(
point
.
getStartOffset
());
ackMsg
.
setConsumerGroup
(
point
.
getCId
());
ackMsg
.
setTopic
(
point
.
getTopic
());
ackMsg
.
setQueueId
(
point
.
getQueueId
());
ackMsg
.
setPopTime
(
point
.
getPopTime
());
msgInner
.
setTopic
(
popMessageProcessor
.
reviveTopic
);
msgInner
.
setBody
(
JSON
.
toJSONString
(
ackMsg
).
getBytes
(
DataConverter
.
charset
));
msgInner
.
setQueueId
(
pointWrapper
.
getReviveQueueId
());
msgInner
.
setTags
(
PopAckConstants
.
ACK_TAG
);
msgInner
.
setBornTimestamp
(
System
.
currentTimeMillis
());
msgInner
.
setBornHost
(
brokerController
.
getStoreHost
());
msgInner
.
setStoreHost
(
brokerController
.
getStoreHost
());
MsgUtil
.
setMessageDeliverTime
(
this
.
brokerController
,
msgInner
,
point
.
getReviveTime
());
msgInner
.
getProperties
().
put
(
MessageConst
.
PROPERTY_UNIQ_CLIENT_MESSAGE_ID_KEYIDX
,
PopMessageProcessor
.
genAckUniqueId
(
ackMsg
));
msgInner
.
setPropertiesString
(
MessageDecoder
.
messageProperties2String
(
msgInner
.
getProperties
()));
PutMessageResult
putMessageResult
=
brokerController
.
getMessageStore
().
putMessage
(
msgInner
);
if
(
putMessageResult
.
getPutMessageStatus
()
!=
PutMessageStatus
.
PUT_OK
&&
putMessageResult
.
getPutMessageStatus
()
!=
PutMessageStatus
.
FLUSH_DISK_TIMEOUT
&&
putMessageResult
.
getPutMessageStatus
()
!=
PutMessageStatus
.
FLUSH_SLAVE_TIMEOUT
&&
putMessageResult
.
getPutMessageStatus
()
!=
PutMessageStatus
.
SLAVE_NOT_AVAILABLE
)
{
POP_LOGGER
.
error
(
"[PopBuffer]put ack to store fail: {}, {}, {}"
,
pointWrapper
,
ackMsg
,
putMessageResult
);
return
false
;
}
if
(
brokerController
.
getBrokerConfig
().
isEnablePopLog
())
{
POP_LOGGER
.
info
(
"[PopBuffer]put ack to store ok: {}, {}, {}"
,
pointWrapper
,
ackMsg
,
putMessageResult
);
}
return
true
;
}
private
boolean
isCkDone
(
PopCheckPointWrapper
pointWrapper
)
{
byte
num
=
pointWrapper
.
getCk
().
getNum
();
for
(
byte
i
=
0
;
i
<
num
;
i
++)
{
if
(!
DataConverter
.
getBit
(
pointWrapper
.
getBits
().
get
(),
i
))
{
return
false
;
}
}
return
true
;
}
private
boolean
isCkDoneForFinish
(
PopCheckPointWrapper
pointWrapper
)
{
byte
num
=
pointWrapper
.
getCk
().
getNum
();
int
bits
=
pointWrapper
.
getBits
().
get
()
^
pointWrapper
.
getToStoreBits
().
get
();
for
(
byte
i
=
0
;
i
<
num
;
i
++)
{
if
(
DataConverter
.
getBit
(
bits
,
i
))
{
return
false
;
}
}
return
true
;
}
public
class
QueueWithTime
<
T
>
{
private
final
LinkedBlockingDeque
<
T
>
queue
;
private
long
time
;
public
QueueWithTime
()
{
this
.
queue
=
new
LinkedBlockingDeque
<>();
this
.
time
=
System
.
currentTimeMillis
();
}
public
void
setTime
(
long
popTime
)
{
this
.
time
=
popTime
;
}
public
long
getTime
()
{
return
time
;
}
public
LinkedBlockingDeque
<
T
>
get
()
{
return
queue
;
}
}
public
class
PopCheckPointWrapper
{
private
final
int
reviveQueueId
;
// -1: not stored, >=0: stored, Long.MAX: storing.
private
volatile
long
reviveQueueOffset
;
private
final
PopCheckPoint
ck
;
// bit for concurrent
private
final
AtomicInteger
bits
;
// bit for stored buffer ak
private
final
AtomicInteger
toStoreBits
;
private
final
long
nextBeginOffset
;
private
final
String
lockKey
;
private
final
String
mergeKey
;
private
final
boolean
justOffset
;
private
volatile
boolean
ckStored
=
false
;
public
PopCheckPointWrapper
(
int
reviveQueueId
,
long
reviveQueueOffset
,
PopCheckPoint
point
,
long
nextBeginOffset
)
{
this
.
reviveQueueId
=
reviveQueueId
;
this
.
reviveQueueOffset
=
reviveQueueOffset
;
this
.
ck
=
point
;
this
.
bits
=
new
AtomicInteger
(
0
);
this
.
toStoreBits
=
new
AtomicInteger
(
0
);
this
.
nextBeginOffset
=
nextBeginOffset
;
this
.
lockKey
=
ck
.
getTopic
()
+
PopAckConstants
.
SPLIT
+
ck
.
getCId
()
+
PopAckConstants
.
SPLIT
+
ck
.
getQueueId
();
this
.
mergeKey
=
point
.
getTopic
()
+
point
.
getCId
()
+
point
.
getQueueId
()
+
point
.
getStartOffset
()
+
point
.
getPopTime
();
this
.
justOffset
=
false
;
}
public
PopCheckPointWrapper
(
int
reviveQueueId
,
long
reviveQueueOffset
,
PopCheckPoint
point
,
long
nextBeginOffset
,
boolean
justOffset
)
{
this
.
reviveQueueId
=
reviveQueueId
;
this
.
reviveQueueOffset
=
reviveQueueOffset
;
this
.
ck
=
point
;
this
.
bits
=
new
AtomicInteger
(
0
);
this
.
toStoreBits
=
new
AtomicInteger
(
0
);
this
.
nextBeginOffset
=
nextBeginOffset
;
this
.
lockKey
=
ck
.
getTopic
()
+
PopAckConstants
.
SPLIT
+
ck
.
getCId
()
+
PopAckConstants
.
SPLIT
+
ck
.
getQueueId
();
this
.
mergeKey
=
point
.
getTopic
()
+
point
.
getCId
()
+
point
.
getQueueId
()
+
point
.
getStartOffset
()
+
point
.
getPopTime
();
this
.
justOffset
=
justOffset
;
}
public
int
getReviveQueueId
()
{
return
reviveQueueId
;
}
public
long
getReviveQueueOffset
()
{
return
reviveQueueOffset
;
}
public
boolean
isCkStored
()
{
return
ckStored
;
}
public
void
setReviveQueueOffset
(
long
reviveQueueOffset
)
{
this
.
reviveQueueOffset
=
reviveQueueOffset
;
}
public
PopCheckPoint
getCk
()
{
return
ck
;
}
public
AtomicInteger
getBits
()
{
return
bits
;
}
public
AtomicInteger
getToStoreBits
()
{
return
toStoreBits
;
}
public
long
getNextBeginOffset
()
{
return
nextBeginOffset
;
}
public
String
getLockKey
()
{
return
lockKey
;
}
public
String
getMergeKey
()
{
return
mergeKey
;
}
public
boolean
isJustOffset
()
{
return
justOffset
;
}
public
void
setCkStored
(
boolean
ckStored
)
{
this
.
ckStored
=
ckStored
;
}
@Override
public
String
toString
()
{
final
StringBuilder
sb
=
new
StringBuilder
(
"CkWrap{"
);
sb
.
append
(
"rq="
).
append
(
reviveQueueId
);
sb
.
append
(
", rqo="
).
append
(
reviveQueueOffset
);
sb
.
append
(
", ck="
).
append
(
ck
);
sb
.
append
(
", bits="
).
append
(
bits
);
sb
.
append
(
", sBits="
).
append
(
toStoreBits
);
sb
.
append
(
", nbo="
).
append
(
nextBeginOffset
);
sb
.
append
(
", cks="
).
append
(
ckStored
);
sb
.
append
(
", jo="
).
append
(
justOffset
);
sb
.
append
(
'}'
);
return
sb
.
toString
();
}
}
}
broker/src/main/java/org/apache/rocketmq/broker/processor/PopMessageProcessor.java
0 → 100644
浏览文件 @
ced6b023
/*
* 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.processor
;
import
com.alibaba.fastjson.JSON
;
import
com.googlecode.concurrentlinkedhashmap.ConcurrentLinkedHashMap
;
import
io.netty.channel.Channel
;
import
io.netty.channel.ChannelFuture
;
import
io.netty.channel.ChannelFutureListener
;
import
io.netty.channel.ChannelHandlerContext
;
import
io.netty.channel.FileRegion
;
import
java.nio.ByteBuffer
;
import
java.util.Iterator
;
import
java.util.List
;
import
java.util.Map.Entry
;
import
java.util.Random
;
import
java.util.concurrent.ConcurrentHashMap
;
import
java.util.concurrent.ConcurrentSkipListSet
;
import
java.util.concurrent.atomic.AtomicBoolean
;
import
java.util.concurrent.atomic.AtomicLong
;
import
org.apache.rocketmq.broker.BrokerController
;
import
org.apache.rocketmq.broker.client.ConsumerGroupInfo
;
import
org.apache.rocketmq.broker.filter.ConsumerFilterData
;
import
org.apache.rocketmq.broker.filter.ConsumerFilterManager
;
import
org.apache.rocketmq.broker.filter.ExpressionMessageFilter
;
import
org.apache.rocketmq.broker.longpolling.PopRequest
;
import
org.apache.rocketmq.broker.pagecache.ManyMessageTransfer
;
import
org.apache.rocketmq.broker.util.MsgUtil
;
import
org.apache.rocketmq.common.KeyBuilder
;
import
org.apache.rocketmq.common.PopAckConstants
;
import
org.apache.rocketmq.common.ServiceThread
;
import
org.apache.rocketmq.common.TopicConfig
;
import
org.apache.rocketmq.common.constant.ConsumeInitMode
;
import
org.apache.rocketmq.common.constant.LoggerName
;
import
org.apache.rocketmq.common.constant.PermName
;
import
org.apache.rocketmq.common.filter.ExpressionType
;
import
org.apache.rocketmq.common.filter.FilterAPI
;
import
org.apache.rocketmq.common.help.FAQUrl
;
import
org.apache.rocketmq.common.message.MessageConst
;
import
org.apache.rocketmq.common.message.MessageDecoder
;
import
org.apache.rocketmq.common.protocol.ResponseCode
;
import
org.apache.rocketmq.common.protocol.header.ExtraInfoUtil
;
import
org.apache.rocketmq.common.protocol.header.PopMessageRequestHeader
;
import
org.apache.rocketmq.common.protocol.header.PopMessageResponseHeader
;
import
org.apache.rocketmq.common.protocol.heartbeat.SubscriptionData
;
import
org.apache.rocketmq.common.subscription.SubscriptionGroupConfig
;
import
org.apache.rocketmq.common.utils.DataConverter
;
import
org.apache.rocketmq.logging.InternalLogger
;
import
org.apache.rocketmq.logging.InternalLoggerFactory
;
import
org.apache.rocketmq.remoting.common.RemotingHelper
;
import
org.apache.rocketmq.remoting.exception.RemotingCommandException
;
import
org.apache.rocketmq.remoting.netty.NettyRequestProcessor
;
import
org.apache.rocketmq.remoting.netty.RequestTask
;
import
org.apache.rocketmq.remoting.protocol.RemotingCommand
;
import
org.apache.rocketmq.store.GetMessageResult
;
import
org.apache.rocketmq.store.GetMessageStatus
;
import
org.apache.rocketmq.store.MessageExtBrokerInner
;
import
org.apache.rocketmq.store.SelectMappedBufferResult
;
import
org.apache.rocketmq.store.pop.AckMsg
;
import
org.apache.rocketmq.store.pop.PopCheckPoint
;
public
class
PopMessageProcessor
implements
NettyRequestProcessor
{
private
static
final
InternalLogger
POP_LOGGER
=
InternalLoggerFactory
.
getLogger
(
LoggerName
.
ROCKETMQ_POP_LOGGER_NAME
);
private
final
BrokerController
brokerController
;
private
Random
random
=
new
Random
(
System
.
currentTimeMillis
());
String
reviveTopic
;
private
static
final
String
BORN_TIME
=
"bornTime"
;
private
static
final
int
POLLING_SUC
=
0
;
private
static
final
int
POLLING_FULL
=
1
;
private
static
final
int
POLLING_TIMEOUT
=
2
;
private
static
final
int
NOT_POLLING
=
3
;
private
ConcurrentHashMap
<
String
,
ConcurrentHashMap
<
String
,
Byte
>>
topicCidMap
;
private
ConcurrentLinkedHashMap
<
String
,
ConcurrentSkipListSet
<
PopRequest
>>
pollingMap
;
private
AtomicLong
totalPollingNum
=
new
AtomicLong
(
0
);
private
PopLongPollingService
popLongPollingService
;
private
PopBufferMergeService
popBufferMergeService
;
private
QueueLockManager
queueLockManager
;
public
PopMessageProcessor
(
final
BrokerController
brokerController
)
{
this
.
brokerController
=
brokerController
;
this
.
reviveTopic
=
PopAckConstants
.
REVIVE_TOPIC
+
this
.
brokerController
.
getBrokerConfig
().
getBrokerClusterName
();
// 100000 topic default, 100000 lru topic + cid + qid
this
.
topicCidMap
=
new
ConcurrentHashMap
<>(
this
.
brokerController
.
getBrokerConfig
().
getPopPollingMapSize
());
this
.
pollingMap
=
new
ConcurrentLinkedHashMap
.
Builder
<
String
,
ConcurrentSkipListSet
<
PopRequest
>>()
.
maximumWeightedCapacity
(
this
.
brokerController
.
getBrokerConfig
().
getPopPollingMapSize
()).
build
();
this
.
popLongPollingService
=
new
PopLongPollingService
();
this
.
queueLockManager
=
new
QueueLockManager
();
this
.
popBufferMergeService
=
new
PopBufferMergeService
(
this
.
brokerController
,
this
);
}
public
PopLongPollingService
getPopLongPollingService
()
{
return
popLongPollingService
;
}
public
PopBufferMergeService
getPopBufferMergeService
()
{
return
this
.
popBufferMergeService
;
}
public
QueueLockManager
getQueueLockManager
()
{
return
queueLockManager
;
}
public
static
String
genAckUniqueId
(
AckMsg
ackMsg
)
{
return
ackMsg
.
getTopic
()
+
PopAckConstants
.
SPLIT
+
ackMsg
.
getQueueId
()
+
PopAckConstants
.
SPLIT
+
ackMsg
.
getAckOffset
()
+
PopAckConstants
.
SPLIT
+
ackMsg
.
getConsumerGroup
()
+
PopAckConstants
.
SPLIT
+
ackMsg
.
getPopTime
()
+
PopAckConstants
.
SPLIT
+
PopAckConstants
.
ACK_TAG
;
}
public
static
String
genCkUniqueId
(
PopCheckPoint
ck
)
{
return
ck
.
getTopic
()
+
PopAckConstants
.
SPLIT
+
ck
.
getQueueId
()
+
PopAckConstants
.
SPLIT
+
ck
.
getStartOffset
()
+
PopAckConstants
.
SPLIT
+
ck
.
getCId
()
+
PopAckConstants
.
SPLIT
+
ck
.
getPopTime
()
+
PopAckConstants
.
SPLIT
+
PopAckConstants
.
CK_TAG
;
}
@Override
public
RemotingCommand
processRequest
(
final
ChannelHandlerContext
ctx
,
RemotingCommand
request
)
throws
RemotingCommandException
{
request
.
addExtField
(
BORN_TIME
,
String
.
valueOf
(
System
.
currentTimeMillis
()));
return
this
.
processRequest
(
ctx
.
channel
(),
request
);
}
@Override
public
boolean
rejectRequest
()
{
return
false
;
}
public
ConcurrentLinkedHashMap
<
String
,
ConcurrentSkipListSet
<
PopRequest
>>
getPollingMap
()
{
return
pollingMap
;
}
public
void
notifyMessageArriving
(
final
String
topic
,
final
int
queueId
)
{
ConcurrentHashMap
<
String
,
Byte
>
cids
=
topicCidMap
.
get
(
topic
);
if
(
cids
==
null
)
{
return
;
}
for
(
Entry
<
String
,
Byte
>
cid
:
cids
.
entrySet
())
{
if
(
queueId
>=
0
)
{
notifyMessageArriving
(
topic
,
cid
.
getKey
(),
-
1
);
}
notifyMessageArriving
(
topic
,
cid
.
getKey
(),
queueId
);
}
}
public
void
notifyMessageArriving
(
final
String
topic
,
final
String
cid
,
final
int
queueId
)
{
ConcurrentSkipListSet
<
PopRequest
>
remotingCommands
=
pollingMap
.
get
(
KeyBuilder
.
buildPollingKey
(
topic
,
cid
,
queueId
));
if
(
remotingCommands
==
null
||
remotingCommands
.
isEmpty
())
{
return
;
}
PopRequest
popRequest
=
remotingCommands
.
pollFirst
();
//clean inactive channel
while
(
popRequest
!=
null
&&
!
popRequest
.
getChannel
().
isActive
())
{
popRequest
=
remotingCommands
.
pollFirst
();
}
if
(
popRequest
==
null
)
{
return
;
}
totalPollingNum
.
decrementAndGet
();
if
(
brokerController
.
getBrokerConfig
().
isEnablePopLog
())
{
POP_LOGGER
.
info
(
"lock release , new msg arrive , wakeUp : {}"
,
popRequest
);
}
wakeUp
(
popRequest
);
}
private
void
wakeUp
(
final
PopRequest
request
)
{
if
(
request
==
null
||
!
request
.
complete
())
{
return
;
}
Runnable
run
=
new
Runnable
()
{
@Override
public
void
run
()
{
try
{
final
RemotingCommand
response
=
PopMessageProcessor
.
this
.
processRequest
(
request
.
getChannel
(),
request
.
getRemotingCommand
());
if
(
response
!=
null
)
{
response
.
setOpaque
(
request
.
getRemotingCommand
().
getOpaque
());
response
.
markResponseType
();
try
{
request
.
getChannel
().
writeAndFlush
(
response
).
addListener
(
new
ChannelFutureListener
()
{
@Override
public
void
operationComplete
(
ChannelFuture
future
)
throws
Exception
{
if
(!
future
.
isSuccess
())
{
POP_LOGGER
.
error
(
"ProcessRequestWrapper response to {} failed"
,
future
.
channel
().
remoteAddress
(),
future
.
cause
());
POP_LOGGER
.
error
(
request
.
toString
());
POP_LOGGER
.
error
(
response
.
toString
());
}
}
});
}
catch
(
Throwable
e
)
{
POP_LOGGER
.
error
(
"ProcessRequestWrapper process request over, but response failed"
,
e
);
POP_LOGGER
.
error
(
request
.
toString
());
POP_LOGGER
.
error
(
response
.
toString
());
}
}
}
catch
(
RemotingCommandException
e1
)
{
POP_LOGGER
.
error
(
"ExecuteRequestWhenWakeup run"
,
e1
);
}
}
};
this
.
brokerController
.
getPullMessageExecutor
().
submit
(
new
RequestTask
(
run
,
request
.
getChannel
(),
request
.
getRemotingCommand
()));
}
private
RemotingCommand
processRequest
(
final
Channel
channel
,
RemotingCommand
request
)
throws
RemotingCommandException
{
RemotingCommand
response
=
RemotingCommand
.
createResponseCommand
(
PopMessageResponseHeader
.
class
);
final
PopMessageResponseHeader
responseHeader
=
(
PopMessageResponseHeader
)
response
.
readCustomHeader
();
final
PopMessageRequestHeader
requestHeader
=
(
PopMessageRequestHeader
)
request
.
decodeCommandCustomHeader
(
PopMessageRequestHeader
.
class
);
StringBuilder
startOffsetInfo
=
new
StringBuilder
(
64
);
StringBuilder
msgOffsetInfo
=
new
StringBuilder
(
64
);
StringBuilder
orderCountInfo
=
null
;
if
(
requestHeader
.
isOrder
())
{
orderCountInfo
=
new
StringBuilder
(
64
);
}
response
.
setOpaque
(
request
.
getOpaque
());
if
(
brokerController
.
getBrokerConfig
().
isEnablePopLog
())
{
POP_LOGGER
.
info
(
"receive PopMessage request command, {}"
,
request
);
}
if
(
requestHeader
.
isTimeoutTooMuch
())
{
response
.
setCode
(
POLLING_TIMEOUT
);
response
.
setRemark
(
String
.
format
(
"the broker[%s] poping message is timeout too much"
,
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] poping message is forbidden"
,
this
.
brokerController
.
getBrokerConfig
().
getBrokerIP1
()));
return
response
;
}
if
(
requestHeader
.
getMaxMsgNums
()
>
32
)
{
response
.
setCode
(
ResponseCode
.
SYSTEM_ERROR
);
response
.
setRemark
(
String
.
format
(
"the broker[%s] poping message's num is greater than 32"
,
this
.
brokerController
.
getBrokerConfig
().
getBrokerIP1
()));
return
response
;
}
TopicConfig
topicConfig
=
this
.
brokerController
.
getTopicConfigManager
().
selectTopicConfig
(
requestHeader
.
getTopic
());
if
(
null
==
topicConfig
)
{
POP_LOGGER
.
error
(
"The topic {} not exist, consumer: {} "
,
requestHeader
.
getTopic
(),
RemotingHelper
.
parseChannelRemoteAddr
(
channel
));
response
.
setCode
(
ResponseCode
.
TOPIC_NOT_EXIST
);
response
.
setRemark
(
String
.
format
(
"topic[%s] not exist, apply first please! %s"
,
requestHeader
.
getTopic
(),
FAQUrl
.
suggestTodo
(
FAQUrl
.
APPLY_TOPIC_URL
)));
return
response
;
}
if
(!
PermName
.
isReadable
(
topicConfig
.
getPerm
()))
{
response
.
setCode
(
ResponseCode
.
NO_PERMISSION
);
response
.
setRemark
(
"the topic["
+
requestHeader
.
getTopic
()
+
"] peeking message is forbidden"
);
return
response
;
}
if
(
requestHeader
.
getQueueId
()
>=
topicConfig
.
getReadQueueNums
())
{
String
errorInfo
=
String
.
format
(
"queueId[%d] is illegal, topic:[%s] topicConfig.readQueueNums:[%d] "
+
"consumer:[%s]"
,
requestHeader
.
getQueueId
(),
requestHeader
.
getTopic
(),
topicConfig
.
getReadQueueNums
(),
channel
.
remoteAddress
());
POP_LOGGER
.
warn
(
errorInfo
);
response
.
setCode
(
ResponseCode
.
SYSTEM_ERROR
);
response
.
setRemark
(
errorInfo
);
return
response
;
}
SubscriptionGroupConfig
subscriptionGroupConfig
=
this
.
brokerController
.
getSubscriptionGroupManager
().
findSubscriptionGroupConfig
(
requestHeader
.
getConsumerGroup
());
if
(
null
==
subscriptionGroupConfig
)
{
response
.
setCode
(
ResponseCode
.
SUBSCRIPTION_GROUP_NOT_EXIST
);
response
.
setRemark
(
String
.
format
(
"subscription group [%s] does not exist, %s"
,
requestHeader
.
getConsumerGroup
(),
FAQUrl
.
suggestTodo
(
FAQUrl
.
SUBSCRIPTION_GROUP_NOT_EXIST
)));
return
response
;
}
ConsumerGroupInfo
consumerGroupInfo
=
this
.
brokerController
.
getConsumerManager
().
getConsumerGroupInfo
(
requestHeader
.
getConsumerGroup
());
if
(
null
==
consumerGroupInfo
)
{
POP_LOGGER
.
warn
(
"the consumer's group info not exist, group: {}"
,
requestHeader
.
getConsumerGroup
());
response
.
setCode
(
ResponseCode
.
SUBSCRIPTION_NOT_EXIST
);
response
.
setRemark
(
"the consumer's group info not exist"
+
FAQUrl
.
suggestTodo
(
FAQUrl
.
SAME_GROUP_DIFFERENT_TOPIC
));
return
response
;
}
if
(!
subscriptionGroupConfig
.
isConsumeEnable
())
{
response
.
setCode
(
ResponseCode
.
NO_PERMISSION
);
response
.
setRemark
(
"subscription group no permission, "
+
requestHeader
.
getConsumerGroup
());
return
response
;
}
ExpressionMessageFilter
messageFilter
=
null
;
if
(
requestHeader
.
getExp
()
!=
null
&&
requestHeader
.
getExp
().
length
()
>
0
)
{
try
{
SubscriptionData
subscriptionData
=
FilterAPI
.
build
(
requestHeader
.
getTopic
(),
requestHeader
.
getExp
(),
requestHeader
.
getExpType
());
ConsumerFilterData
consumerFilterData
=
null
;
if
(!
ExpressionType
.
isTagType
(
subscriptionData
.
getExpressionType
()))
{
consumerFilterData
=
ConsumerFilterManager
.
build
(
requestHeader
.
getTopic
(),
requestHeader
.
getConsumerGroup
(),
requestHeader
.
getExp
(),
requestHeader
.
getExpType
(),
System
.
currentTimeMillis
()
);
if
(
consumerFilterData
==
null
)
{
POP_LOGGER
.
warn
(
"Parse the consumer's subscription[{}] failed, group: {}"
,
requestHeader
.
getExp
(),
requestHeader
.
getConsumerGroup
());
response
.
setCode
(
ResponseCode
.
SUBSCRIPTION_PARSE_FAILED
);
response
.
setRemark
(
"parse the consumer's subscription failed"
);
return
response
;
}
}
messageFilter
=
new
ExpressionMessageFilter
(
subscriptionData
,
consumerFilterData
,
brokerController
.
getConsumerFilterManager
());
}
catch
(
Exception
e
)
{
POP_LOGGER
.
warn
(
"Parse the consumer's subscription[{}] error, group: {}"
,
requestHeader
.
getExp
(),
requestHeader
.
getConsumerGroup
());
response
.
setCode
(
ResponseCode
.
SUBSCRIPTION_PARSE_FAILED
);
response
.
setRemark
(
"parse the consumer's subscription failed"
);
return
response
;
}
}
int
randomQ
=
random
.
nextInt
(
100
);
int
reviveQid
;
if
(
requestHeader
.
isOrder
())
{
reviveQid
=
KeyBuilder
.
POP_ORDER_REVIVE_QUEUE
;
}
else
{
reviveQid
=
randomQ
%
this
.
brokerController
.
getBrokerConfig
().
getReviveQueueNum
();
}
GetMessageResult
getMessageResult
=
new
GetMessageResult
();
long
restNum
=
0
;
boolean
needRetry
=
randomQ
%
5
==
0
;
long
popTime
=
System
.
currentTimeMillis
();
if
(
needRetry
&&
!
requestHeader
.
isOrder
())
{
TopicConfig
retryTopicConfig
=
this
.
brokerController
.
getTopicConfigManager
().
selectTopicConfig
(
KeyBuilder
.
buildPopRetryTopic
(
requestHeader
.
getTopic
(),
requestHeader
.
getConsumerGroup
()));
if
(
retryTopicConfig
!=
null
)
{
for
(
int
i
=
0
;
i
<
retryTopicConfig
.
getReadQueueNums
();
i
++)
{
int
queueId
=
(
randomQ
+
i
)
%
retryTopicConfig
.
getReadQueueNums
();
restNum
=
popMsgFromQueue
(
true
,
getMessageResult
,
requestHeader
,
queueId
,
restNum
,
reviveQid
,
channel
,
popTime
,
messageFilter
,
startOffsetInfo
,
msgOffsetInfo
,
orderCountInfo
);
}
}
}
if
(
requestHeader
.
getQueueId
()
<
0
)
{
// read all queue
for
(
int
i
=
0
;
i
<
topicConfig
.
getReadQueueNums
();
i
++)
{
int
queueId
=
(
randomQ
+
i
)
%
topicConfig
.
getReadQueueNums
();
restNum
=
popMsgFromQueue
(
false
,
getMessageResult
,
requestHeader
,
queueId
,
restNum
,
reviveQid
,
channel
,
popTime
,
messageFilter
,
startOffsetInfo
,
msgOffsetInfo
,
orderCountInfo
);
}
}
else
{
int
queueId
=
requestHeader
.
getQueueId
();
restNum
=
popMsgFromQueue
(
false
,
getMessageResult
,
requestHeader
,
queueId
,
restNum
,
reviveQid
,
channel
,
popTime
,
messageFilter
,
startOffsetInfo
,
msgOffsetInfo
,
orderCountInfo
);
}
// if not full , fetch retry again
if
(!
needRetry
&&
getMessageResult
.
getMessageMapedList
().
size
()
<
requestHeader
.
getMaxMsgNums
()
&&
!
requestHeader
.
isOrder
())
{
TopicConfig
retryTopicConfig
=
this
.
brokerController
.
getTopicConfigManager
().
selectTopicConfig
(
KeyBuilder
.
buildPopRetryTopic
(
requestHeader
.
getTopic
(),
requestHeader
.
getConsumerGroup
()));
if
(
retryTopicConfig
!=
null
)
{
for
(
int
i
=
0
;
i
<
retryTopicConfig
.
getReadQueueNums
();
i
++)
{
int
queueId
=
(
randomQ
+
i
)
%
retryTopicConfig
.
getReadQueueNums
();
restNum
=
popMsgFromQueue
(
true
,
getMessageResult
,
requestHeader
,
queueId
,
restNum
,
reviveQid
,
channel
,
popTime
,
messageFilter
,
startOffsetInfo
,
msgOffsetInfo
,
orderCountInfo
);
}
}
}
if
(!
getMessageResult
.
getMessageBufferList
().
isEmpty
())
{
response
.
setCode
(
ResponseCode
.
SUCCESS
);
getMessageResult
.
setStatus
(
GetMessageStatus
.
FOUND
);
if
(
restNum
>
0
)
{
// all queue pop can not notify specified queue pop, and vice versa
notifyMessageArriving
(
requestHeader
.
getTopic
(),
requestHeader
.
getConsumerGroup
(),
requestHeader
.
getQueueId
());
}
}
else
{
int
pollingResult
=
polling
(
channel
,
request
,
requestHeader
);
if
(
POLLING_SUC
==
pollingResult
)
{
return
null
;
}
else
if
(
POLLING_FULL
==
pollingResult
)
{
response
.
setCode
(
ResponseCode
.
POLLING_FULL
);
}
else
{
response
.
setCode
(
ResponseCode
.
POLLING_TIMEOUT
);
}
getMessageResult
.
setStatus
(
GetMessageStatus
.
NO_MESSAGE_IN_QUEUE
);
}
responseHeader
.
setInvisibleTime
(
requestHeader
.
getInvisibleTime
());
responseHeader
.
setPopTime
(
popTime
);
responseHeader
.
setReviveQid
(
reviveQid
);
responseHeader
.
setRestNum
(
restNum
);
responseHeader
.
setStartOffsetInfo
(
startOffsetInfo
.
toString
());
responseHeader
.
setMsgOffsetInfo
(
msgOffsetInfo
.
toString
());
if
(
requestHeader
.
isOrder
()
&&
orderCountInfo
!=
null
)
{
responseHeader
.
setOrderCountInfo
(
orderCountInfo
.
toString
());
}
response
.
setRemark
(
getMessageResult
.
getStatus
().
name
());
switch
(
response
.
getCode
())
{
case
ResponseCode
.
SUCCESS
:
if
(
this
.
brokerController
.
getBrokerConfig
().
isTransferMsgByHeap
())
{
final
long
beginTimeMills
=
this
.
brokerController
.
getMessageStore
().
now
();
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
(
new
ChannelFutureListener
()
{
@Override
public
void
operationComplete
(
ChannelFuture
future
)
throws
Exception
{
tmpGetMessageResult
.
release
();
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
();
}
response
=
null
;
}
break
;
default
:
assert
false
;
}
return
response
;
}
private
long
popMsgFromQueue
(
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
topic
=
isRetry
?
KeyBuilder
.
buildPopRetryTopic
(
requestHeader
.
getTopic
(),
requestHeader
.
getConsumerGroup
())
:
requestHeader
.
getTopic
();
String
lockKey
=
topic
+
PopAckConstants
.
SPLIT
+
requestHeader
.
getConsumerGroup
()
+
PopAckConstants
.
SPLIT
+
queueId
;
boolean
isOrder
=
requestHeader
.
isOrder
();
long
offset
=
getPopOffset
(
topic
,
requestHeader
,
queueId
,
false
,
lockKey
);
if
(!
queueLockManager
.
tryLock
(
lockKey
))
{
restNum
=
this
.
brokerController
.
getMessageStore
().
getMaxOffsetInQueue
(
topic
,
queueId
)
-
offset
+
restNum
;
return
restNum
;
}
offset
=
getPopOffset
(
topic
,
requestHeader
,
queueId
,
true
,
lockKey
);
GetMessageResult
getMessageTmpResult
;
try
{
if
(
isOrder
&&
brokerController
.
getConsumerOrderInfoManager
().
checkBlock
(
topic
,
requestHeader
.
getConsumerGroup
(),
queueId
,
requestHeader
.
getInvisibleTime
()))
{
return
this
.
brokerController
.
getMessageStore
().
getMaxOffsetInQueue
(
topic
,
queueId
)
-
offset
+
restNum
;
}
if
(
getMessageResult
.
getMessageMapedList
().
size
()
>=
requestHeader
.
getMaxMsgNums
())
{
restNum
=
this
.
brokerController
.
getMessageStore
().
getMaxOffsetInQueue
(
topic
,
queueId
)
-
offset
+
restNum
;
return
restNum
;
}
getMessageTmpResult
=
this
.
brokerController
.
getMessageStore
().
getMessage
(
requestHeader
.
getConsumerGroup
()
,
topic
,
queueId
,
offset
,
requestHeader
.
getMaxMsgNums
()
-
getMessageResult
.
getMessageMapedList
().
size
(),
messageFilter
);
// maybe store offset is not correct.
if
(
getMessageTmpResult
==
null
||
GetMessageStatus
.
OFFSET_TOO_SMALL
.
equals
(
getMessageTmpResult
.
getStatus
())
||
GetMessageStatus
.
OFFSET_OVERFLOW_BADLY
.
equals
(
getMessageTmpResult
.
getStatus
())
||
GetMessageStatus
.
OFFSET_FOUND_NULL
.
equals
(
getMessageTmpResult
.
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.
POP_LOGGER
.
warn
(
"Pop initial offset, because store is no correct, {}, {}->{}"
,
lockKey
,
offset
,
getMessageTmpResult
!=
null
?
getMessageTmpResult
.
getNextBeginOffset
()
:
"null"
);
offset
=
getMessageTmpResult
!=
null
?
getMessageTmpResult
.
getNextBeginOffset
()
:
0
;
this
.
brokerController
.
getConsumerOffsetManager
().
commitOffset
(
channel
.
remoteAddress
().
toString
(),
requestHeader
.
getConsumerGroup
(),
topic
,
queueId
,
offset
);
getMessageTmpResult
=
this
.
brokerController
.
getMessageStore
().
getMessage
(
requestHeader
.
getConsumerGroup
(),
topic
,
queueId
,
offset
,
requestHeader
.
getMaxMsgNums
()
-
getMessageResult
.
getMessageMapedList
().
size
(),
messageFilter
);
}
restNum
=
getMessageTmpResult
.
getMaxOffset
()
-
getMessageTmpResult
.
getNextBeginOffset
()
+
restNum
;
if
(!
getMessageTmpResult
.
getMessageMapedList
().
isEmpty
())
{
this
.
brokerController
.
getBrokerStatsManager
().
incBrokerGetNums
(
getMessageTmpResult
.
getMessageCount
());
this
.
brokerController
.
getBrokerStatsManager
().
incGroupGetNums
(
requestHeader
.
getConsumerGroup
(),
topic
,
getMessageTmpResult
.
getMessageCount
());
this
.
brokerController
.
getBrokerStatsManager
().
incGroupGetSize
(
requestHeader
.
getConsumerGroup
(),
topic
,
getMessageTmpResult
.
getBufferTotalSize
());
if
(
isOrder
)
{
int
count
=
brokerController
.
getConsumerOrderInfoManager
().
update
(
topic
,
requestHeader
.
getConsumerGroup
(),
queueId
,
getMessageTmpResult
.
getMessageQueueOffset
());
this
.
brokerController
.
getConsumerOffsetManager
().
commitOffset
(
channel
.
remoteAddress
().
toString
(),
requestHeader
.
getConsumerGroup
(),
topic
,
queueId
,
offset
);
ExtraInfoUtil
.
buildOrderCountInfo
(
orderCountInfo
,
isRetry
,
queueId
,
count
);
}
else
{
appendCheckPoint
(
requestHeader
,
topic
,
reviveQid
,
queueId
,
offset
,
getMessageTmpResult
,
popTime
);
}
ExtraInfoUtil
.
buildStartOffsetInfo
(
startOffsetInfo
,
isRetry
,
queueId
,
offset
);
ExtraInfoUtil
.
buildMsgOffsetInfo
(
msgOffsetInfo
,
isRetry
,
queueId
,
getMessageTmpResult
.
getMessageQueueOffset
());
}
else
if
((
GetMessageStatus
.
NO_MATCHED_MESSAGE
.
equals
(
getMessageTmpResult
.
getStatus
())
||
GetMessageStatus
.
OFFSET_FOUND_NULL
.
equals
(
getMessageTmpResult
.
getStatus
())
||
GetMessageStatus
.
MESSAGE_WAS_REMOVING
.
equals
(
getMessageTmpResult
.
getStatus
())
||
GetMessageStatus
.
NO_MATCHED_LOGIC_QUEUE
.
equals
(
getMessageTmpResult
.
getStatus
()))
&&
getMessageTmpResult
.
getNextBeginOffset
()
>
-
1
)
{
popBufferMergeService
.
addCkMock
(
requestHeader
.
getConsumerGroup
(),
topic
,
queueId
,
offset
,
requestHeader
.
getInvisibleTime
(),
popTime
,
reviveQid
,
getMessageTmpResult
.
getNextBeginOffset
());
}
}
finally
{
queueLockManager
.
unLock
(
lockKey
);
}
if
(
getMessageTmpResult
!=
null
)
{
for
(
SelectMappedBufferResult
mapedBuffer
:
getMessageTmpResult
.
getMessageMapedList
())
{
getMessageResult
.
addMessage
(
mapedBuffer
);
}
}
return
restNum
;
}
private
long
getPopOffset
(
String
topic
,
PopMessageRequestHeader
requestHeader
,
int
queueId
,
boolean
init
,
String
lockKey
)
{
long
offset
=
this
.
brokerController
.
getConsumerOffsetManager
().
queryOffset
(
requestHeader
.
getConsumerGroup
(),
topic
,
queueId
);
if
(
offset
<
0
)
{
if
(
ConsumeInitMode
.
MIN
==
requestHeader
.
getInitMode
())
{
offset
=
this
.
brokerController
.
getMessageStore
().
getMinOffsetInQueue
(
topic
,
queueId
);
}
else
{
// pop last one,then commit offset.
offset
=
this
.
brokerController
.
getMessageStore
().
getMaxOffsetInQueue
(
topic
,
queueId
)
-
1
;
// max & no consumer offset
if
(
offset
<
0
)
{
offset
=
0
;
}
if
(
init
)
{
this
.
brokerController
.
getConsumerOffsetManager
().
commitOffset
(
"getPopOffset"
,
requestHeader
.
getConsumerGroup
(),
topic
,
queueId
,
offset
);
}
}
}
long
bufferOffset
=
this
.
popBufferMergeService
.
getLatestOffset
(
lockKey
);
if
(
bufferOffset
<
0
)
{
return
offset
;
}
else
{
return
bufferOffset
>
offset
?
bufferOffset
:
offset
;
}
}
/**
* @param channel
* @param remotingCommand
* @param requestHeader
* @return
*/
private
int
polling
(
final
Channel
channel
,
RemotingCommand
remotingCommand
,
final
PopMessageRequestHeader
requestHeader
)
{
if
(
requestHeader
.
getPollTime
()
<=
0
||
this
.
popLongPollingService
.
isStopped
())
{
return
NOT_POLLING
;
}
ConcurrentHashMap
<
String
,
Byte
>
cids
=
topicCidMap
.
get
(
requestHeader
.
getTopic
());
if
(
cids
==
null
)
{
cids
=
new
ConcurrentHashMap
<>();
ConcurrentHashMap
<
String
,
Byte
>
old
=
topicCidMap
.
putIfAbsent
(
requestHeader
.
getTopic
(),
cids
);
if
(
old
!=
null
)
{
cids
=
old
;
}
}
cids
.
putIfAbsent
(
requestHeader
.
getConsumerGroup
(),
Byte
.
MIN_VALUE
);
long
expired
=
requestHeader
.
getBornTime
()
+
requestHeader
.
getPollTime
();
final
PopRequest
request
=
new
PopRequest
(
remotingCommand
,
channel
,
expired
);
boolean
isFull
=
totalPollingNum
.
get
()
>=
this
.
brokerController
.
getBrokerConfig
().
getMaxPopPollingSize
();
if
(
isFull
)
{
POP_LOGGER
.
info
(
"polling {}, result POLLING_FULL, total:{}"
,
remotingCommand
,
totalPollingNum
.
get
());
return
POLLING_FULL
;
}
boolean
isTimeout
=
request
.
isTimeout
();
if
(
isTimeout
)
{
if
(
brokerController
.
getBrokerConfig
().
isEnablePopLog
())
{
POP_LOGGER
.
info
(
"polling {}, result POLLING_TIMEOUT"
,
remotingCommand
);
}
return
POLLING_TIMEOUT
;
}
String
key
=
KeyBuilder
.
buildPollingKey
(
requestHeader
.
getTopic
(),
requestHeader
.
getConsumerGroup
(),
requestHeader
.
getQueueId
());
ConcurrentSkipListSet
<
PopRequest
>
queue
=
pollingMap
.
get
(
key
);
if
(
queue
==
null
)
{
queue
=
new
ConcurrentSkipListSet
<>(
PopRequest
.
COMPARATOR
);
ConcurrentSkipListSet
<
PopRequest
>
old
=
pollingMap
.
putIfAbsent
(
key
,
queue
);
if
(
old
!=
null
)
{
queue
=
old
;
}
}
else
{
// check size
int
size
=
queue
.
size
();
if
(
size
>
brokerController
.
getBrokerConfig
().
getPopPollingSize
())
{
POP_LOGGER
.
info
(
"polling {}, result POLLING_FULL, singleSize:{}"
,
remotingCommand
,
size
);
return
POLLING_FULL
;
}
}
if
(
queue
.
add
(
request
))
{
totalPollingNum
.
incrementAndGet
();
if
(
brokerController
.
getBrokerConfig
().
isEnablePopLog
())
{
POP_LOGGER
.
info
(
"polling {}, result POLLING_SUC"
,
remotingCommand
);
}
return
POLLING_SUC
;
}
else
{
POP_LOGGER
.
info
(
"polling {}, result POLLING_FULL, add fail, {}"
,
request
,
queue
);
return
POLLING_FULL
;
}
}
public
final
MessageExtBrokerInner
buildCkMsg
(
final
PopCheckPoint
ck
,
final
int
reviveQid
)
{
MessageExtBrokerInner
msgInner
=
new
MessageExtBrokerInner
();
msgInner
.
setTopic
(
reviveTopic
);
msgInner
.
setBody
(
JSON
.
toJSONString
(
ck
).
getBytes
(
DataConverter
.
charset
));
msgInner
.
setQueueId
(
reviveQid
);
msgInner
.
setTags
(
PopAckConstants
.
CK_TAG
);
msgInner
.
setBornTimestamp
(
System
.
currentTimeMillis
());
msgInner
.
setBornHost
(
this
.
brokerController
.
getStoreHost
());
msgInner
.
setStoreHost
(
this
.
brokerController
.
getStoreHost
());
MsgUtil
.
setMessageDeliverTime
(
this
.
brokerController
,
msgInner
,
ck
.
getReviveTime
()
-
PopAckConstants
.
ackTimeInterval
);
msgInner
.
getProperties
().
put
(
MessageConst
.
PROPERTY_UNIQ_CLIENT_MESSAGE_ID_KEYIDX
,
genCkUniqueId
(
ck
));
msgInner
.
setPropertiesString
(
MessageDecoder
.
messageProperties2String
(
msgInner
.
getProperties
()));
return
msgInner
;
}
private
void
appendCheckPoint
(
final
PopMessageRequestHeader
requestHeader
,
final
String
topic
,
final
int
reviveQid
,
final
int
queueId
,
final
long
offset
,
final
GetMessageResult
getMessageTmpResult
,
final
long
popTime
)
{
// add check point msg to revive log
final
PopCheckPoint
ck
=
new
PopCheckPoint
();
ck
.
setBitMap
(
0
);
ck
.
setNum
((
byte
)
getMessageTmpResult
.
getMessageMapedList
().
size
());
ck
.
setPopTime
(
popTime
);
ck
.
setInvisibleTime
(
requestHeader
.
getInvisibleTime
());
ck
.
getStartOffset
(
offset
);
ck
.
setCId
(
requestHeader
.
getConsumerGroup
());
ck
.
setTopic
(
topic
);
ck
.
setQueueId
((
byte
)
queueId
);
for
(
Long
msgQueueOffset
:
getMessageTmpResult
.
getMessageQueueOffset
())
{
ck
.
addDiff
((
int
)
(
msgQueueOffset
-
offset
));
}
final
boolean
addBufferSuc
=
this
.
popBufferMergeService
.
addCk
(
ck
,
reviveQid
,
-
1
,
getMessageTmpResult
.
getNextBeginOffset
()
);
if
(
addBufferSuc
)
{
return
;
}
this
.
popBufferMergeService
.
addCkJustOffset
(
ck
,
reviveQid
,
-
1
,
getMessageTmpResult
.
getNextBeginOffset
()
);
}
private
byte
[]
readGetMessageResult
(
final
GetMessageResult
getMessageResult
,
final
String
group
,
final
String
topic
,
final
int
queueId
)
{
final
ByteBuffer
byteBuffer
=
ByteBuffer
.
allocate
(
getMessageResult
.
getBufferTotalSize
());
long
storeTimestamp
=
0
;
try
{
List
<
ByteBuffer
>
messageBufferList
=
getMessageResult
.
getMessageBufferList
();
for
(
ByteBuffer
bb
:
messageBufferList
)
{
byteBuffer
.
put
(
bb
);
storeTimestamp
=
bb
.
getLong
(
MessageDecoder
.
MESSAGE_STORE_TIMESTAMP_POSITION
);
}
}
finally
{
getMessageResult
.
release
();
}
this
.
brokerController
.
getBrokerStatsManager
().
recordDiskFallBehindTime
(
group
,
topic
,
queueId
,
this
.
brokerController
.
getMessageStore
().
now
()
-
storeTimestamp
);
return
byteBuffer
.
array
();
}
public
class
PopLongPollingService
extends
ServiceThread
{
private
long
lastCleanTime
=
0
;
@Override
public
String
getServiceName
()
{
return
"PopLongPollingService"
;
}
private
void
cleanUnusedResource
()
{
try
{
{
Iterator
<
Entry
<
String
,
ConcurrentHashMap
<
String
,
Byte
>>>
topicCidMapIter
=
topicCidMap
.
entrySet
().
iterator
();
while
(
topicCidMapIter
.
hasNext
())
{
Entry
<
String
,
ConcurrentHashMap
<
String
,
Byte
>>
entry
=
topicCidMapIter
.
next
();
String
topic
=
entry
.
getKey
();
if
(
brokerController
.
getTopicConfigManager
().
selectTopicConfig
(
topic
)
==
null
)
{
POP_LOGGER
.
info
(
"remove not exit topic {} in topicCidMap!"
,
topic
);
topicCidMapIter
.
remove
();
continue
;
}
Iterator
<
Entry
<
String
,
Byte
>>
cidMapIter
=
entry
.
getValue
().
entrySet
().
iterator
();
while
(
cidMapIter
.
hasNext
())
{
Entry
<
String
,
Byte
>
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
);
cidMapIter
.
remove
();
}
}
}
}
{
Iterator
<
Entry
<
String
,
ConcurrentSkipListSet
<
PopRequest
>>>
pollingMapIter
=
pollingMap
.
entrySet
().
iterator
();
while
(
pollingMapIter
.
hasNext
())
{
Entry
<
String
,
ConcurrentSkipListSet
<
PopRequest
>>
entry
=
pollingMapIter
.
next
();
if
(
entry
.
getKey
()
==
null
)
{
continue
;
}
String
[]
keyArray
=
entry
.
getKey
().
split
(
PopAckConstants
.
SPLIT
);
if
(
keyArray
==
null
||
keyArray
.
length
!=
3
)
{
continue
;
}
String
topic
=
keyArray
[
0
];
String
cid
=
keyArray
[
1
];
if
(
brokerController
.
getTopicConfigManager
().
selectTopicConfig
(
topic
)
==
null
)
{
POP_LOGGER
.
info
(
"remove not exit 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
);
pollingMapIter
.
remove
();
continue
;
}
}
}
}
catch
(
Throwable
e
)
{
POP_LOGGER
.
error
(
"cleanUnusedResource"
,
e
);
}
lastCleanTime
=
System
.
currentTimeMillis
();
}
@Override
public
void
run
()
{
int
i
=
0
;
while
(!
this
.
stopped
)
{
try
{
this
.
waitForRunning
(
20
);
i
++;
if
(
pollingMap
.
isEmpty
())
{
continue
;
}
long
tmpTotalPollingNum
=
0
;
Iterator
<
Entry
<
String
,
ConcurrentSkipListSet
<
PopRequest
>>>
pollingMapIterator
=
pollingMap
.
entrySet
().
iterator
();
while
(
pollingMapIterator
.
hasNext
())
{
Entry
<
String
,
ConcurrentSkipListSet
<
PopRequest
>>
entry
=
pollingMapIterator
.
next
();
String
key
=
entry
.
getKey
();
ConcurrentSkipListSet
<
PopRequest
>
popQ
=
entry
.
getValue
();
if
(
popQ
==
null
)
{
continue
;
}
PopRequest
first
;
do
{
first
=
popQ
.
pollFirst
();
if
(
first
==
null
)
{
break
;
}
if
(!
first
.
isTimeout
())
{
if
(
popQ
.
add
(
first
))
{
break
;
}
else
{
POP_LOGGER
.
info
(
"polling, add fail again: {}"
,
first
);
}
}
if
(
brokerController
.
getBrokerConfig
().
isEnablePopLog
())
{
POP_LOGGER
.
info
(
"timeout , wakeUp polling : {}"
,
first
);
}
totalPollingNum
.
decrementAndGet
();
wakeUp
(
first
);
}
while
(
true
);
if
(
i
>=
100
)
{
long
tmpPollingNum
=
popQ
.
size
();
tmpTotalPollingNum
=
tmpTotalPollingNum
+
tmpPollingNum
;
if
(
tmpPollingNum
>
100
)
{
POP_LOGGER
.
info
(
"polling queue {} , size={} "
,
key
,
tmpPollingNum
);
}
}
}
if
(
i
>=
100
)
{
POP_LOGGER
.
info
(
"pollingMapSize={},tmpTotalSize={},atomicTotalSize={},diffSize={}"
,
pollingMap
.
size
(),
tmpTotalPollingNum
,
totalPollingNum
.
get
(),
Math
.
abs
(
totalPollingNum
.
get
()
-
tmpTotalPollingNum
));
totalPollingNum
.
set
(
tmpTotalPollingNum
);
i
=
0
;
}
// clean unused
if
(
lastCleanTime
==
0
||
System
.
currentTimeMillis
()
-
lastCleanTime
>
5
*
60
*
1000
)
{
cleanUnusedResource
();
}
}
catch
(
Throwable
e
)
{
POP_LOGGER
.
error
(
"checkPolling error"
,
e
);
}
}
// clean all;
try
{
Iterator
<
Entry
<
String
,
ConcurrentSkipListSet
<
PopRequest
>>>
pollingMapIterator
=
pollingMap
.
entrySet
().
iterator
();
while
(
pollingMapIterator
.
hasNext
())
{
Entry
<
String
,
ConcurrentSkipListSet
<
PopRequest
>>
entry
=
pollingMapIterator
.
next
();
ConcurrentSkipListSet
<
PopRequest
>
popQ
=
entry
.
getValue
();
PopRequest
first
;
while
((
first
=
popQ
.
pollFirst
())
!=
null
)
{
wakeUp
(
first
);
}
}
}
catch
(
Throwable
e
)
{
}
}
}
static
class
TimedLock
{
private
final
AtomicBoolean
lock
;
private
volatile
long
lockTime
;
public
TimedLock
()
{
this
.
lock
=
new
AtomicBoolean
(
true
);
this
.
lockTime
=
System
.
currentTimeMillis
();
}
public
boolean
tryLock
()
{
boolean
ret
=
lock
.
compareAndSet
(
true
,
false
);
if
(
ret
)
{
this
.
lockTime
=
System
.
currentTimeMillis
();
return
true
;
}
else
{
return
false
;
}
}
public
void
unLock
()
{
lock
.
set
(
true
);
}
public
boolean
isLock
()
{
return
!
lock
.
get
();
}
public
long
getLockTime
()
{
return
lockTime
;
}
}
public
static
class
QueueLockManager
extends
ServiceThread
{
private
ConcurrentHashMap
<
String
,
TimedLock
>
expiredLocalCache
=
new
ConcurrentHashMap
<>(
100000
);
public
boolean
tryLock
(
String
key
)
{
TimedLock
timedLock
=
expiredLocalCache
.
get
(
key
);
if
(
timedLock
==
null
)
{
TimedLock
old
=
expiredLocalCache
.
putIfAbsent
(
key
,
new
TimedLock
());
if
(
old
!=
null
)
{
return
false
;
}
else
{
timedLock
=
expiredLocalCache
.
get
(
key
);
}
}
if
(
timedLock
==
null
)
{
return
false
;
}
return
timedLock
.
tryLock
();
}
/**
* is not thread safe, may cause duplicate lock
*
* @param usedExpireMillis
* @return
*/
public
int
cleanUnusedLock
(
final
long
usedExpireMillis
)
{
Iterator
<
Entry
<
String
,
TimedLock
>>
iterator
=
expiredLocalCache
.
entrySet
().
iterator
();
int
total
=
0
;
while
(
iterator
.
hasNext
())
{
Entry
<
String
,
TimedLock
>
entry
=
iterator
.
next
();
if
(
System
.
currentTimeMillis
()
-
entry
.
getValue
().
getLockTime
()
>
usedExpireMillis
)
{
iterator
.
remove
();
POP_LOGGER
.
info
(
"Remove unused queue lock: {}, {}, {}"
,
entry
.
getKey
(),
entry
.
getValue
().
getLockTime
(),
entry
.
getValue
().
isLock
());
}
total
++;
}
return
total
;
}
public
void
unLock
(
String
key
)
{
TimedLock
timedLock
=
expiredLocalCache
.
get
(
key
);
if
(
timedLock
!=
null
)
{
timedLock
.
unLock
();
}
}
@Override
public
String
getServiceName
()
{
return
"QueueLockManager"
;
}
@Override
public
void
run
()
{
while
(!
isStopped
())
{
try
{
this
.
waitForRunning
(
60000
);
int
count
=
cleanUnusedLock
(
60000
);
POP_LOGGER
.
info
(
"QueueLockSize={}"
,
count
);
}
catch
(
Exception
e
)
{
PopMessageProcessor
.
POP_LOGGER
.
error
(
"QueueLockManager run error"
,
e
);
}
}
}
}
}
broker/src/main/java/org/apache/rocketmq/broker/processor/PopReviveService.java
0 → 100644
浏览文件 @
ced6b023
/*
* 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.processor
;
import
com.alibaba.fastjson.JSON
;
import
java.nio.ByteBuffer
;
import
java.util.ArrayList
;
import
java.util.Collections
;
import
java.util.Comparator
;
import
java.util.HashMap
;
import
java.util.List
;
import
org.apache.rocketmq.broker.BrokerController
;
import
org.apache.rocketmq.broker.util.MsgUtil
;
import
org.apache.rocketmq.client.consumer.PullResult
;
import
org.apache.rocketmq.client.consumer.PullStatus
;
import
org.apache.rocketmq.common.KeyBuilder
;
import
org.apache.rocketmq.common.MixAll
;
import
org.apache.rocketmq.common.PopAckConstants
;
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.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.utils.DataConverter
;
import
org.apache.rocketmq.logging.InternalLogger
;
import
org.apache.rocketmq.logging.InternalLoggerFactory
;
import
org.apache.rocketmq.store.AppendMessageStatus
;
import
org.apache.rocketmq.store.GetMessageResult
;
import
org.apache.rocketmq.store.MessageExtBrokerInner
;
import
org.apache.rocketmq.store.PutMessageResult
;
import
org.apache.rocketmq.store.config.BrokerRole
;
import
org.apache.rocketmq.store.pop.AckMsg
;
import
org.apache.rocketmq.store.pop.PopCheckPoint
;
public
class
PopReviveService
extends
ServiceThread
{
private
static
final
InternalLogger
POP_LOGGER
=
InternalLoggerFactory
.
getLogger
(
LoggerName
.
ROCKETMQ_POP_LOGGER_NAME
);
private
int
queueId
;
private
BrokerController
brokerController
;
private
String
reviveTopic
;
private
static
volatile
boolean
isMaster
=
false
;
public
PopReviveService
(
int
queueId
,
BrokerController
brokerController
,
String
reviveTopic
)
{
super
();
this
.
queueId
=
queueId
;
this
.
brokerController
=
brokerController
;
this
.
reviveTopic
=
reviveTopic
;
}
@Override
public
String
getServiceName
()
{
return
"PopReviveService_"
+
this
.
queueId
;
}
private
boolean
checkMaster
()
{
return
brokerController
.
getMessageStoreConfig
().
getBrokerRole
()
!=
BrokerRole
.
SLAVE
;
}
private
boolean
checkAndSetMaster
()
{
isMaster
=
checkMaster
();
return
isMaster
;
}
private
void
reviveRetry
(
PopCheckPoint
popCheckPoint
,
MessageExt
messageExt
)
throws
Exception
{
if
(!
checkAndSetMaster
())
{
POP_LOGGER
.
info
(
"slave skip retry , revive topic={}, reviveQueueId={}"
,
reviveTopic
,
queueId
);
return
;
}
MessageExtBrokerInner
msgInner
=
new
MessageExtBrokerInner
();
if
(!
popCheckPoint
.
getTopic
().
startsWith
(
MixAll
.
RETRY_GROUP_TOPIC_PREFIX
))
{
msgInner
.
setTopic
(
KeyBuilder
.
buildPopRetryTopic
(
popCheckPoint
.
getTopic
(),
popCheckPoint
.
getCId
()));
}
else
{
msgInner
.
setTopic
(
popCheckPoint
.
getTopic
());
}
msgInner
.
setBody
(
messageExt
.
getBody
());
msgInner
.
setQueueId
(
0
);
if
(
messageExt
.
getTags
()
!=
null
)
{
msgInner
.
setTags
(
messageExt
.
getTags
());
}
else
{
MessageAccessor
.
setProperties
(
msgInner
,
new
HashMap
<
String
,
String
>());
}
msgInner
.
setBornTimestamp
(
messageExt
.
getBornTimestamp
());
msgInner
.
setBornHost
(
brokerController
.
getStoreHost
());
msgInner
.
setStoreHost
(
brokerController
.
getStoreHost
());
msgInner
.
setReconsumeTimes
(
messageExt
.
getReconsumeTimes
()
+
1
);
msgInner
.
getProperties
().
putAll
(
messageExt
.
getProperties
());
if
(
messageExt
.
getReconsumeTimes
()
==
0
||
msgInner
.
getProperties
().
get
(
MessageConst
.
PROPERTY_FIRST_POP_TIME
)
==
null
)
{
msgInner
.
getProperties
().
put
(
MessageConst
.
PROPERTY_FIRST_POP_TIME
,
String
.
valueOf
(
popCheckPoint
.
getPopTime
()));
}
msgInner
.
setPropertiesString
(
MessageDecoder
.
messageProperties2String
(
msgInner
.
getProperties
()));
addRetryTopicIfNoExit
(
msgInner
.
getTopic
());
PutMessageResult
putMessageResult
=
brokerController
.
getMessageStore
().
putMessage
(
msgInner
);
if
(
brokerController
.
getBrokerConfig
().
isEnablePopLog
())
{
POP_LOGGER
.
info
(
"reviveQueueId={},retry msg , ck={}, msg queueId {}, offset {}, reviveDelay={}, result is {} "
,
queueId
,
popCheckPoint
,
messageExt
.
getQueueId
(),
messageExt
.
getQueueOffset
(),
(
System
.
currentTimeMillis
()
-
popCheckPoint
.
getReviveTime
())
/
1000
,
putMessageResult
);
}
if
(
putMessageResult
.
getAppendMessageResult
()
==
null
||
putMessageResult
.
getAppendMessageResult
().
getStatus
()
!=
AppendMessageStatus
.
PUT_OK
)
{
throw
new
Exception
(
"reviveQueueId="
+
queueId
+
",revive error ,msg is :"
+
msgInner
);
}
this
.
brokerController
.
getBrokerStatsManager
().
incBrokerPutNums
(
1
);
this
.
brokerController
.
getBrokerStatsManager
().
incTopicPutNums
(
msgInner
.
getTopic
());
this
.
brokerController
.
getBrokerStatsManager
().
incTopicPutSize
(
msgInner
.
getTopic
(),
putMessageResult
.
getAppendMessageResult
().
getWroteBytes
());
if
(
brokerController
.
getPopMessageProcessor
()
!=
null
)
{
brokerController
.
getPopMessageProcessor
().
notifyMessageArriving
(
KeyBuilder
.
parseNormalTopic
(
popCheckPoint
.
getTopic
(),
popCheckPoint
.
getCId
()),
popCheckPoint
.
getCId
(),
-
1
);
}
}
private
boolean
addRetryTopicIfNoExit
(
String
topic
)
{
TopicConfig
topicConfig
=
brokerController
.
getTopicConfigManager
().
selectTopicConfig
(
topic
);
if
(
topicConfig
!=
null
)
{
return
true
;
}
topicConfig
=
new
TopicConfig
(
topic
);
topicConfig
.
setReadQueueNums
(
PopAckConstants
.
retryQueueNum
);
topicConfig
.
setWriteQueueNums
(
PopAckConstants
.
retryQueueNum
);
topicConfig
.
setTopicFilterType
(
TopicFilterType
.
SINGLE_TAG
);
topicConfig
.
setPerm
(
6
);
topicConfig
.
setTopicSysFlag
(
0
);
brokerController
.
getTopicConfigManager
().
updateTopicConfig
(
topicConfig
);
return
true
;
}
private
List
<
MessageExt
>
getReviveMessage
(
long
offset
,
int
queueId
)
{
PullResult
pullResult
=
getMessage
(
PopAckConstants
.
REVIVE_GROUP
,
reviveTopic
,
queueId
,
offset
,
32
);
if
(
pullResult
==
null
)
{
return
null
;
}
if
(
reachTail
(
pullResult
,
offset
))
{
POP_LOGGER
.
info
(
"reviveQueueId={}, reach tail,offset {}"
,
queueId
,
offset
);
}
else
if
(
pullResult
.
getPullStatus
()
==
PullStatus
.
OFFSET_ILLEGAL
||
pullResult
.
getPullStatus
()
==
PullStatus
.
NO_MATCHED_MSG
)
{
POP_LOGGER
.
error
(
"reviveQueueId={}, OFFSET_ILLEGAL {}, result is {}"
,
queueId
,
offset
,
pullResult
);
if
(!
checkAndSetMaster
())
{
POP_LOGGER
.
info
(
"slave skip offset correct topic={}, reviveQueueId={}"
,
reviveTopic
,
queueId
);
return
null
;
}
brokerController
.
getConsumerOffsetManager
().
commitOffset
(
PopAckConstants
.
LOCAL_HOST
,
PopAckConstants
.
REVIVE_GROUP
,
reviveTopic
,
queueId
,
pullResult
.
getNextBeginOffset
()
-
1
);
}
return
pullResult
.
getMsgFoundList
();
}
private
boolean
reachTail
(
PullResult
pullResult
,
long
offset
)
{
return
pullResult
.
getPullStatus
()
==
PullStatus
.
NO_NEW_MSG
||
(
pullResult
.
getPullStatus
()
==
PullStatus
.
OFFSET_ILLEGAL
&&
offset
==
pullResult
.
getMaxOffset
());
}
private
MessageExt
getBizMessage
(
String
topic
,
long
offset
,
int
queueId
)
{
final
GetMessageResult
getMessageTmpResult
=
brokerController
.
getMessageStore
().
getMessage
(
PopAckConstants
.
REVIVE_GROUP
,
topic
,
queueId
,
offset
,
1
,
null
);
List
<
MessageExt
>
list
=
decodeMsgList
(
getMessageTmpResult
);
if
(
list
==
null
||
list
.
isEmpty
())
{
POP_LOGGER
.
warn
(
"can not get msg , topic {}, offset {}, queueId {}, result is {}"
,
topic
,
offset
,
queueId
,
getMessageTmpResult
);
return
null
;
}
else
{
return
list
.
get
(
0
);
}
}
public
PullResult
getMessage
(
String
group
,
String
topic
,
int
queueId
,
long
offset
,
int
nums
)
{
GetMessageResult
getMessageResult
=
brokerController
.
getMessageStore
().
getMessage
(
group
,
topic
,
queueId
,
offset
,
nums
,
null
);
if
(
getMessageResult
!=
null
)
{
PullStatus
pullStatus
=
PullStatus
.
NO_NEW_MSG
;
List
<
MessageExt
>
foundList
=
null
;
switch
(
getMessageResult
.
getStatus
())
{
case
FOUND:
pullStatus
=
PullStatus
.
FOUND
;
foundList
=
decodeMsgList
(
getMessageResult
);
brokerController
.
getBrokerStatsManager
().
incGroupGetNums
(
group
,
topic
,
getMessageResult
.
getMessageCount
());
brokerController
.
getBrokerStatsManager
().
incGroupGetSize
(
group
,
topic
,
getMessageResult
.
getBufferTotalSize
());
brokerController
.
getBrokerStatsManager
().
incBrokerGetNums
(
getMessageResult
.
getMessageCount
());
brokerController
.
getBrokerStatsManager
().
recordDiskFallBehindTime
(
group
,
topic
,
queueId
,
brokerController
.
getMessageStore
().
now
()
-
foundList
.
get
(
foundList
.
size
()
-
1
).
getStoreTimestamp
());
break
;
case
NO_MATCHED_MESSAGE:
pullStatus
=
PullStatus
.
NO_MATCHED_MSG
;
POP_LOGGER
.
warn
(
"no matched message. GetMessageStatus={}, topic={}, groupId={}, requestOffset={}"
,
getMessageResult
.
getStatus
(),
topic
,
group
,
offset
);
break
;
case
NO_MESSAGE_IN_QUEUE:
pullStatus
=
PullStatus
.
NO_NEW_MSG
;
POP_LOGGER
.
warn
(
"no new message. GetMessageStatus={}, topic={}, groupId={}, requestOffset={}"
,
getMessageResult
.
getStatus
(),
topic
,
group
,
offset
);
break
;
case
MESSAGE_WAS_REMOVING:
case
NO_MATCHED_LOGIC_QUEUE:
case
OFFSET_FOUND_NULL:
case
OFFSET_OVERFLOW_BADLY:
case
OFFSET_OVERFLOW_ONE:
case
OFFSET_TOO_SMALL:
pullStatus
=
PullStatus
.
OFFSET_ILLEGAL
;
POP_LOGGER
.
warn
(
"offset illegal. GetMessageStatus={}, topic={}, groupId={}, requestOffset={}"
,
getMessageResult
.
getStatus
(),
topic
,
group
,
offset
);
break
;
default
:
assert
false
;
break
;
}
return
new
PullResult
(
pullStatus
,
getMessageResult
.
getNextBeginOffset
(),
getMessageResult
.
getMinOffset
(),
getMessageResult
.
getMaxOffset
(),
foundList
);
}
else
{
POP_LOGGER
.
error
(
"get message from store return null. topic={}, groupId={}, requestOffset={}"
,
topic
,
group
,
offset
);
return
null
;
}
}
private
List
<
MessageExt
>
decodeMsgList
(
GetMessageResult
getMessageResult
)
{
List
<
MessageExt
>
foundList
=
new
ArrayList
<>();
try
{
List
<
ByteBuffer
>
messageBufferList
=
getMessageResult
.
getMessageBufferList
();
if
(
messageBufferList
!=
null
)
{
for
(
int
i
=
0
;
i
<
messageBufferList
.
size
();
i
++)
{
ByteBuffer
bb
=
messageBufferList
.
get
(
i
);
if
(
bb
==
null
)
{
POP_LOGGER
.
error
(
"bb is null {}"
,
getMessageResult
);
continue
;
}
MessageExt
msgExt
=
MessageDecoder
.
decode
(
bb
);
if
(
msgExt
==
null
)
{
POP_LOGGER
.
error
(
"decode msgExt is null {}"
,
getMessageResult
);
continue
;
}
// use CQ offset, not offset in Message
msgExt
.
setQueueOffset
(
getMessageResult
.
getMessageQueueOffset
().
get
(
i
));
foundList
.
add
(
msgExt
);
}
}
}
finally
{
getMessageResult
.
release
();
}
return
foundList
;
}
private
void
consumeReviveMessage
(
ConsumeReviveObj
consumeReviveObj
)
{
HashMap
<
String
,
PopCheckPoint
>
map
=
consumeReviveObj
.
map
;
long
startScanTime
=
System
.
currentTimeMillis
();
long
endTime
=
0
;
long
oldOffset
=
brokerController
.
getConsumerOffsetManager
().
queryOffset
(
PopAckConstants
.
REVIVE_GROUP
,
reviveTopic
,
queueId
);
consumeReviveObj
.
oldOffset
=
oldOffset
;
POP_LOGGER
.
info
(
"reviveQueueId={}, old offset is {} "
,
queueId
,
oldOffset
);
long
offset
=
oldOffset
+
1
;
long
firstRt
=
0
;
// offset self amend
while
(
true
)
{
if
(!
checkAndSetMaster
())
{
POP_LOGGER
.
info
(
"slave skip scan , revive topic={}, reviveQueueId={}"
,
reviveTopic
,
queueId
);
break
;
}
List
<
MessageExt
>
messageExts
=
getReviveMessage
(
offset
,
queueId
);
if
(
messageExts
==
null
||
messageExts
.
isEmpty
())
{
break
;
}
if
(
System
.
currentTimeMillis
()
-
startScanTime
>
brokerController
.
getBrokerConfig
().
getReviveScanTime
())
{
POP_LOGGER
.
info
(
"reviveQueueId={}, scan timeout "
,
queueId
);
break
;
}
for
(
MessageExt
messageExt
:
messageExts
)
{
if
(
PopAckConstants
.
CK_TAG
.
equals
(
messageExt
.
getTags
()))
{
String
raw
=
new
String
(
messageExt
.
getBody
(),
DataConverter
.
charset
);
if
(
brokerController
.
getBrokerConfig
().
isEnablePopLog
())
{
POP_LOGGER
.
info
(
"reviveQueueId={},find ck, offset:{}, raw : {}"
,
messageExt
.
getQueueId
(),
messageExt
.
getQueueOffset
(),
raw
);
}
PopCheckPoint
point
=
JSON
.
parseObject
(
raw
,
PopCheckPoint
.
class
);
if
(
point
.
getTopic
()
==
null
||
point
.
getCId
()
==
null
)
{
continue
;
}
map
.
put
(
point
.
getTopic
()
+
point
.
getCId
()
+
point
.
getQueueId
()
+
point
.
getStartOffset
()
+
point
.
getPopTime
(),
point
);
point
.
setReviveOffset
(
messageExt
.
getQueueOffset
());
if
(
firstRt
==
0
)
{
firstRt
=
point
.
getReviveTime
();
}
}
else
if
(
PopAckConstants
.
ACK_TAG
.
equals
(
messageExt
.
getTags
()))
{
String
raw
=
new
String
(
messageExt
.
getBody
(),
DataConverter
.
charset
);
if
(
brokerController
.
getBrokerConfig
().
isEnablePopLog
())
{
POP_LOGGER
.
info
(
"reviveQueueId={},find ack, offset:{}, raw : {}"
,
messageExt
.
getQueueId
(),
messageExt
.
getQueueOffset
(),
raw
);
}
AckMsg
ackMsg
=
JSON
.
parseObject
(
raw
,
AckMsg
.
class
);
PopCheckPoint
point
=
map
.
get
(
ackMsg
.
getTopic
()
+
ackMsg
.
getConsumerGroup
()
+
ackMsg
.
getQueueId
()
+
ackMsg
.
getStartOffset
()
+
ackMsg
.
getPopTime
());
if
(
point
==
null
)
{
continue
;
}
int
indexOfAck
=
point
.
indexOfAck
(
ackMsg
.
getAckOffset
());
if
(
indexOfAck
>
-
1
)
{
point
.
setBitMap
(
DataConverter
.
setBit
(
point
.
getBitMap
(),
indexOfAck
,
true
));
}
else
{
POP_LOGGER
.
error
(
"invalid ack index, {}, {}"
,
ackMsg
,
point
);
}
}
long
deliverTime
=
MsgUtil
.
getMessageDeliverTime
(
this
.
brokerController
,
messageExt
);
if
(
deliverTime
>
endTime
)
{
endTime
=
deliverTime
;
}
}
offset
=
offset
+
messageExts
.
size
();
}
consumeReviveObj
.
endTime
=
endTime
;
}
private
void
mergeAndRevive
(
ConsumeReviveObj
consumeReviveObj
)
throws
Throwable
{
ArrayList
<
PopCheckPoint
>
sortList
=
consumeReviveObj
.
genSortList
();
POP_LOGGER
.
info
(
"reviveQueueId={},ck listSize={}"
,
queueId
,
sortList
.
size
());
if
(
sortList
.
size
()
!=
0
)
{
POP_LOGGER
.
info
(
"reviveQueueId={}, 1st ck, startOffset={}, reviveOffset={} ; last ck, startOffset={}, reviveOffset={}"
,
queueId
,
sortList
.
get
(
0
).
getStartOffset
(),
sortList
.
get
(
0
).
getReviveOffset
(),
sortList
.
get
(
sortList
.
size
()
-
1
).
getStartOffset
(),
sortList
.
get
(
sortList
.
size
()
-
1
).
getReviveOffset
());
}
long
newOffset
=
consumeReviveObj
.
oldOffset
;
for
(
PopCheckPoint
popCheckPoint
:
sortList
)
{
if
(!
checkAndSetMaster
())
{
POP_LOGGER
.
info
(
"slave skip ck process , revive topic={}, reviveQueueId={}"
,
reviveTopic
,
queueId
);
break
;
}
if
(
consumeReviveObj
.
endTime
-
popCheckPoint
.
getReviveTime
()
<=
(
PopAckConstants
.
ackTimeInterval
+
PopAckConstants
.
SECOND
))
{
break
;
}
// check normal topic, skip ck , if normal topic is not exist
String
normalTopic
=
KeyBuilder
.
parseNormalTopic
(
popCheckPoint
.
getTopic
(),
popCheckPoint
.
getCId
());
if
(
brokerController
.
getTopicConfigManager
().
selectTopicConfig
(
normalTopic
)
==
null
)
{
POP_LOGGER
.
warn
(
"reviveQueueId={},can not get normal topic {} , then continue "
,
queueId
,
popCheckPoint
.
getTopic
());
newOffset
=
popCheckPoint
.
getReviveOffset
();
continue
;
}
if
(
null
==
brokerController
.
getSubscriptionGroupManager
().
findSubscriptionGroupConfig
(
popCheckPoint
.
getCId
()))
{
POP_LOGGER
.
warn
(
"reviveQueueId={},can not get cid {} , then continue "
,
queueId
,
popCheckPoint
.
getCId
());
newOffset
=
popCheckPoint
.
getReviveOffset
();
continue
;
}
reviveMsgFromCk
(
popCheckPoint
);
newOffset
=
popCheckPoint
.
getReviveOffset
();
}
if
(
newOffset
>
consumeReviveObj
.
oldOffset
)
{
if
(!
checkAndSetMaster
())
{
POP_LOGGER
.
info
(
"slave skip commit, revive topic={}, reviveQueueId={}"
,
reviveTopic
,
queueId
);
return
;
}
brokerController
.
getConsumerOffsetManager
().
commitOffset
(
PopAckConstants
.
LOCAL_HOST
,
PopAckConstants
.
REVIVE_GROUP
,
reviveTopic
,
queueId
,
newOffset
);
}
consumeReviveObj
.
newOffset
=
newOffset
;
}
private
void
reviveMsgFromCk
(
PopCheckPoint
popCheckPoint
)
throws
Throwable
{
for
(
int
j
=
0
;
j
<
popCheckPoint
.
getNum
();
j
++)
{
if
(
DataConverter
.
getBit
(
popCheckPoint
.
getBitMap
(),
j
))
{
continue
;
}
// retry msg
long
msgOffset
=
popCheckPoint
.
ackOffsetByIndex
((
byte
)
j
);
MessageExt
messageExt
=
getBizMessage
(
popCheckPoint
.
getTopic
(),
msgOffset
,
popCheckPoint
.
getQueueId
());
if
(
messageExt
==
null
)
{
POP_LOGGER
.
warn
(
"reviveQueueId={},can not get biz msg topic is {}, offset is {} , then continue "
,
queueId
,
popCheckPoint
.
getTopic
(),
msgOffset
);
continue
;
}
//skip ck from last epoch
if
(
popCheckPoint
.
getPopTime
()
<
messageExt
.
getStoreTimestamp
())
{
POP_LOGGER
.
warn
(
"reviveQueueId={},skip ck from last epoch {}"
,
queueId
,
popCheckPoint
);
continue
;
}
reviveRetry
(
popCheckPoint
,
messageExt
);
}
}
@Override
public
void
run
()
{
int
slow
=
1
;
while
(!
this
.
isStopped
())
{
try
{
if
(
System
.
currentTimeMillis
()
<
brokerController
.
getShouldStartTime
())
{
POP_LOGGER
.
info
(
"PopReviveService Ready to run after {}"
,
brokerController
.
getShouldStartTime
());
this
.
waitForRunning
(
1000
);
continue
;
}
this
.
waitForRunning
(
brokerController
.
getBrokerConfig
().
getReviveInterval
());
if
(!
checkAndSetMaster
())
{
POP_LOGGER
.
info
(
"slave skip start revive topic={}, reviveQueueId={}"
,
reviveTopic
,
queueId
);
continue
;
}
POP_LOGGER
.
info
(
"start revive topic={}, reviveQueueId={}"
,
reviveTopic
,
queueId
);
ConsumeReviveObj
consumeReviveObj
=
new
ConsumeReviveObj
();
consumeReviveMessage
(
consumeReviveObj
);
if
(!
checkAndSetMaster
())
{
POP_LOGGER
.
info
(
"slave skip scan , revive topic={}, reviveQueueId={}"
,
reviveTopic
,
queueId
);
continue
;
}
mergeAndRevive
(
consumeReviveObj
);
ArrayList
<
PopCheckPoint
>
sortList
=
consumeReviveObj
.
sortList
;
long
delay
=
0
;
if
(
sortList
!=
null
&&
!
sortList
.
isEmpty
())
{
delay
=
(
System
.
currentTimeMillis
()
-
sortList
.
get
(
0
).
getReviveTime
())
/
1000
;
slow
=
1
;
}
POP_LOGGER
.
info
(
"reviveQueueId={},revive finish,old offset is {}, new offset is {}, ckDelay={} "
,
queueId
,
consumeReviveObj
.
oldOffset
,
consumeReviveObj
.
newOffset
,
delay
);
if
(
sortList
==
null
||
sortList
.
isEmpty
())
{
POP_LOGGER
.
info
(
"reviveQueueId={},has no new msg ,take a rest {}"
,
queueId
,
slow
);
this
.
waitForRunning
(
slow
*
brokerController
.
getBrokerConfig
().
getReviveInterval
());
if
(
slow
<
brokerController
.
getBrokerConfig
().
getReviveMaxSlow
())
{
slow
++;
}
}
}
catch
(
Throwable
e
)
{
POP_LOGGER
.
error
(
"reviveQueueId="
+
queueId
+
",revive error"
,
e
);
}
}
}
static
class
ConsumeReviveObj
{
HashMap
<
String
,
PopCheckPoint
>
map
=
new
HashMap
<>();
ArrayList
<
PopCheckPoint
>
sortList
;
long
oldOffset
;
long
endTime
;
long
newOffset
;
ArrayList
<
PopCheckPoint
>
genSortList
()
{
if
(
sortList
!=
null
)
{
return
sortList
;
}
sortList
=
new
ArrayList
<>(
map
.
values
());
Collections
.
sort
(
sortList
,
new
Comparator
<
PopCheckPoint
>()
{
@Override
public
int
compare
(
PopCheckPoint
o1
,
PopCheckPoint
o2
)
{
return
(
int
)
(
o1
.
getReviveOffset
()
-
o2
.
getReviveOffset
());
}
});
return
sortList
;
}
}
}
broker/src/main/java/org/apache/rocketmq/broker/processor/QueryAssignmentProcessor.java
0 → 100644
浏览文件 @
ced6b023
/*
* 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.processor
;
import
io.netty.channel.ChannelHandlerContext
;
import
java.util.ArrayList
;
import
java.util.Collections
;
import
java.util.HashSet
;
import
java.util.List
;
import
java.util.Set
;
import
java.util.concurrent.ConcurrentHashMap
;
import
org.apache.rocketmq.broker.BrokerController
;
import
org.apache.rocketmq.broker.client.ConsumerGroupInfo
;
import
org.apache.rocketmq.broker.loadbalance.AssignmentManager
;
import
org.apache.rocketmq.broker.loadbalance.MessageRequestModeManager
;
import
org.apache.rocketmq.client.consumer.AllocateMessageQueueStrategy
;
import
org.apache.rocketmq.client.consumer.rebalance.AllocateMessageQueueAveragely
;
import
org.apache.rocketmq.client.consumer.rebalance.AllocateMessageQueueAveragelyByCircle
;
import
org.apache.rocketmq.common.MixAll
;
import
org.apache.rocketmq.common.constant.LoggerName
;
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.common.protocol.RequestCode
;
import
org.apache.rocketmq.common.protocol.ResponseCode
;
import
org.apache.rocketmq.common.protocol.body.QueryAssignmentRequestBody
;
import
org.apache.rocketmq.common.protocol.body.QueryAssignmentResponseBody
;
import
org.apache.rocketmq.common.protocol.body.SetMessageRequestModeRequestBody
;
import
org.apache.rocketmq.common.protocol.heartbeat.MessageModel
;
import
org.apache.rocketmq.logging.InternalLogger
;
import
org.apache.rocketmq.logging.InternalLoggerFactory
;
import
org.apache.rocketmq.remoting.common.RemotingHelper
;
import
org.apache.rocketmq.remoting.exception.RemotingCommandException
;
import
org.apache.rocketmq.remoting.netty.NettyRequestProcessor
;
import
org.apache.rocketmq.remoting.protocol.RemotingCommand
;
public
class
QueryAssignmentProcessor
implements
NettyRequestProcessor
{
private
static
final
InternalLogger
log
=
InternalLoggerFactory
.
getLogger
(
LoggerName
.
BROKER_LOGGER_NAME
);
private
final
BrokerController
brokerController
;
private
final
ConcurrentHashMap
<
String
,
AllocateMessageQueueStrategy
>
name2LoadStrategy
=
new
ConcurrentHashMap
<
String
,
AllocateMessageQueueStrategy
>();
private
MessageRequestModeManager
messageRequestModeManager
;
public
QueryAssignmentProcessor
(
final
BrokerController
brokerController
)
{
this
.
brokerController
=
brokerController
;
//register strategy
//NOTE: init with broker's log instead of init with ClientLogger.getLog();
AllocateMessageQueueAveragely
allocateMessageQueueAveragely
=
new
AllocateMessageQueueAveragely
(
log
);
name2LoadStrategy
.
put
(
allocateMessageQueueAveragely
.
getName
(),
allocateMessageQueueAveragely
);
AllocateMessageQueueAveragelyByCircle
allocateMessageQueueAveragelyByCircle
=
new
AllocateMessageQueueAveragelyByCircle
(
log
);
name2LoadStrategy
.
put
(
allocateMessageQueueAveragelyByCircle
.
getName
(),
allocateMessageQueueAveragelyByCircle
);
this
.
messageRequestModeManager
=
new
MessageRequestModeManager
(
brokerController
);
this
.
messageRequestModeManager
.
load
();
}
@Override
public
RemotingCommand
processRequest
(
ChannelHandlerContext
ctx
,
RemotingCommand
request
)
throws
RemotingCommandException
{
switch
(
request
.
getCode
())
{
case
RequestCode
.
QUERY_ASSIGNMENT
:
return
this
.
queryAssignment
(
ctx
,
request
);
case
RequestCode
.
SET_MESSAGE_REQUEST_MODE
:
return
this
.
setMessageRequestMode
(
ctx
,
request
);
default
:
break
;
}
return
null
;
}
@Override
public
boolean
rejectRequest
()
{
return
false
;
}
/**
*
*/
private
RemotingCommand
queryAssignment
(
ChannelHandlerContext
ctx
,
RemotingCommand
request
)
throws
RemotingCommandException
{
final
QueryAssignmentRequestBody
requestBody
=
QueryAssignmentRequestBody
.
decode
(
request
.
getBody
(),
QueryAssignmentRequestBody
.
class
);
final
String
topic
=
requestBody
.
getTopic
();
final
String
consumerGroup
=
requestBody
.
getConsumerGroup
();
final
String
clientId
=
requestBody
.
getClientId
();
final
MessageModel
messageModel
=
requestBody
.
getMessageModel
();
final
String
strategyName
=
requestBody
.
getStrategyName
();
final
RemotingCommand
response
=
RemotingCommand
.
createResponseCommand
(
null
);
final
QueryAssignmentResponseBody
responseBody
=
new
QueryAssignmentResponseBody
();
SetMessageRequestModeRequestBody
setMessageRequestModeRequestBody
=
this
.
messageRequestModeManager
.
getMessageRequestMode
(
topic
,
consumerGroup
);
if
(
setMessageRequestModeRequestBody
==
null
)
{
setMessageRequestModeRequestBody
=
new
SetMessageRequestModeRequestBody
();
setMessageRequestModeRequestBody
.
setTopic
(
topic
);
setMessageRequestModeRequestBody
.
setConsumerGroup
(
consumerGroup
);
if
(
topic
.
startsWith
(
MixAll
.
RETRY_GROUP_TOPIC_PREFIX
))
{
// retry topic must be pull mode
setMessageRequestModeRequestBody
.
setMode
(
MessageRequestMode
.
PULL
);
}
else
{
setMessageRequestModeRequestBody
.
setMode
(
brokerController
.
getBrokerConfig
().
getDefaultMessageRequestMode
());
}
if
(
setMessageRequestModeRequestBody
.
getMode
()
==
MessageRequestMode
.
POP
)
{
setMessageRequestModeRequestBody
.
setPopShareQueueNum
(
brokerController
.
getBrokerConfig
().
getDefaultPopShareQueueNum
());
}
}
Set
<
MessageQueue
>
messageQueues
=
doLoadBalance
(
topic
,
consumerGroup
,
clientId
,
messageModel
,
strategyName
,
setMessageRequestModeRequestBody
,
ctx
);
Set
<
MessageQueueAssignment
>
assignments
=
null
;
if
(
messageQueues
!=
null
)
{
assignments
=
new
HashSet
<
MessageQueueAssignment
>();
for
(
MessageQueue
messageQueue
:
messageQueues
)
{
MessageQueueAssignment
messageQueueAssignment
=
new
MessageQueueAssignment
();
messageQueueAssignment
.
setMessageQueue
(
messageQueue
);
if
(
setMessageRequestModeRequestBody
!=
null
)
{
messageQueueAssignment
.
setMode
(
setMessageRequestModeRequestBody
.
getMode
());
}
assignments
.
add
(
messageQueueAssignment
);
}
}
responseBody
.
setMessageQueueAssignments
(
assignments
);
response
.
setBody
(
responseBody
.
encode
());
response
.
setCode
(
ResponseCode
.
SUCCESS
);
response
.
setRemark
(
null
);
return
response
;
}
/**
* Returns empty set means the client should clear all load assigned to it before, null means invalid result and the
* client should skip the update logic
*
* @param topic
* @param consumerGroup
* @param clientId
* @param messageModel
* @param strategyName
* @return the MessageQueues assigned to this client
*/
private
Set
<
MessageQueue
>
doLoadBalance
(
final
String
topic
,
final
String
consumerGroup
,
final
String
clientId
,
final
MessageModel
messageModel
,
final
String
strategyName
,
SetMessageRequestModeRequestBody
setMessageRequestModeRequestBody
,
final
ChannelHandlerContext
ctx
)
{
Set
<
MessageQueue
>
assignedQueueSet
=
null
;
AssignmentManager
assignmentManager
=
brokerController
.
getAssignmentManager
();
switch
(
messageModel
)
{
case
BROADCASTING:
{
assignedQueueSet
=
assignmentManager
.
getTopicSubscribeInfo
(
topic
);
if
(
assignedQueueSet
==
null
)
{
log
.
warn
(
"QueryLoad: no assignment for group[{}], the topic[{}] does not exist."
,
consumerGroup
,
topic
);
}
break
;
}
case
CLUSTERING:
{
Set
<
MessageQueue
>
mqSet
=
assignmentManager
.
getTopicSubscribeInfo
(
topic
);
if
(
null
==
mqSet
)
{
if
(!
topic
.
startsWith
(
MixAll
.
RETRY_GROUP_TOPIC_PREFIX
))
{
log
.
warn
(
"QueryLoad: no assignment for group[{}], the topic[{}] does not exist."
,
consumerGroup
,
topic
);
}
return
null
;
}
if
(!
brokerController
.
getBrokerConfig
().
isServerLoadBalancerEnabled
())
{
return
mqSet
;
}
List
<
String
>
cidAll
=
null
;
ConsumerGroupInfo
consumerGroupInfo
=
this
.
brokerController
.
getConsumerManager
().
getConsumerGroupInfo
(
consumerGroup
);
if
(
consumerGroupInfo
!=
null
)
{
cidAll
=
consumerGroupInfo
.
getAllClientId
();
}
if
(
null
==
cidAll
)
{
log
.
warn
(
"QueryLoad: no assignment for group[{}] topic[{}], get consumer id list failed"
,
consumerGroup
,
topic
);
return
null
;
}
List
<
MessageQueue
>
mqAll
=
new
ArrayList
<
MessageQueue
>();
mqAll
.
addAll
(
mqSet
);
Collections
.
sort
(
mqAll
);
Collections
.
sort
(
cidAll
);
List
<
MessageQueue
>
allocateResult
=
null
;
try
{
AllocateMessageQueueStrategy
allocateMessageQueueStrategy
=
name2LoadStrategy
.
get
(
strategyName
);
if
(
null
==
allocateMessageQueueStrategy
)
{
log
.
warn
(
"QueryLoad: unsupported strategy [{}], {}"
,
consumerGroup
,
RemotingHelper
.
parseChannelRemoteAddr
(
ctx
.
channel
()));
return
null
;
}
if
(
setMessageRequestModeRequestBody
!=
null
&&
setMessageRequestModeRequestBody
.
getMode
()
==
MessageRequestMode
.
POP
)
{
if
(
setMessageRequestModeRequestBody
.
getPopShareQueueNum
()
<=
0
)
{
//each client pop all messagequeue
allocateResult
=
new
ArrayList
<>(
mqAll
.
size
());
for
(
MessageQueue
mq
:
mqAll
)
{
//must create new MessageQueue in case of change cache in AssignmentManager
MessageQueue
newMq
=
new
MessageQueue
(
mq
.
getTopic
(),
mq
.
getBrokerName
(),
-
1
);
allocateResult
.
add
(
newMq
);
}
}
else
{
if
(
cidAll
.
size
()
<=
mqAll
.
size
())
{
//consumer working in pop mode could share the MessageQueues assigned to the N (N = popWorkGroupSize) consumer following it in the cid list
allocateResult
=
allocateMessageQueueStrategy
.
allocate
(
consumerGroup
,
clientId
,
mqAll
,
cidAll
);
int
index
=
cidAll
.
indexOf
(
clientId
);
if
(
index
>=
0
)
{
for
(
int
i
=
1
;
i
<=
setMessageRequestModeRequestBody
.
getPopShareQueueNum
();
i
++)
{
index
++;
index
=
index
%
cidAll
.
size
();
List
<
MessageQueue
>
tmp
=
allocateMessageQueueStrategy
.
allocate
(
consumerGroup
,
cidAll
.
get
(
index
),
mqAll
,
cidAll
);
allocateResult
.
addAll
(
tmp
);
}
}
}
else
{
//make sure each cid is assigned
allocateResult
=
allocate
(
consumerGroup
,
clientId
,
mqAll
,
cidAll
);
}
}
}
else
{
allocateResult
=
allocateMessageQueueStrategy
.
allocate
(
consumerGroup
,
clientId
,
mqAll
,
cidAll
);
}
}
catch
(
Throwable
e
)
{
log
.
error
(
"QueryLoad: no assignment for group[{}] topic[{}], allocate message queue exception. strategy name: {}, ex: {}"
,
consumerGroup
,
topic
,
strategyName
,
e
);
return
null
;
}
assignedQueueSet
=
new
HashSet
<
MessageQueue
>();
if
(
allocateResult
!=
null
)
{
assignedQueueSet
.
addAll
(
allocateResult
);
}
break
;
}
default
:
break
;
}
return
assignedQueueSet
;
}
private
List
<
MessageQueue
>
allocate
(
String
consumerGroup
,
String
currentCID
,
List
<
MessageQueue
>
mqAll
,
List
<
String
>
cidAll
)
{
if
(
currentCID
==
null
||
currentCID
.
length
()
<
1
)
{
throw
new
IllegalArgumentException
(
"currentCID is empty"
);
}
if
(
mqAll
==
null
||
mqAll
.
isEmpty
())
{
throw
new
IllegalArgumentException
(
"mqAll is null or mqAll empty"
);
}
if
(
cidAll
==
null
||
cidAll
.
isEmpty
())
{
throw
new
IllegalArgumentException
(
"cidAll is null or cidAll empty"
);
}
List
<
MessageQueue
>
result
=
new
ArrayList
<
MessageQueue
>();
if
(!
cidAll
.
contains
(
currentCID
))
{
log
.
info
(
"[BUG] ConsumerGroup: {} The consumerId: {} not in cidAll: {}"
,
consumerGroup
,
currentCID
,
cidAll
);
return
result
;
}
int
index
=
cidAll
.
indexOf
(
currentCID
);
result
.
add
(
mqAll
.
get
(
index
%
mqAll
.
size
()));
return
result
;
}
private
RemotingCommand
setMessageRequestMode
(
ChannelHandlerContext
ctx
,
RemotingCommand
request
)
throws
RemotingCommandException
{
final
RemotingCommand
response
=
RemotingCommand
.
createResponseCommand
(
null
);
final
SetMessageRequestModeRequestBody
requestBody
=
SetMessageRequestModeRequestBody
.
decode
(
request
.
getBody
(),
SetMessageRequestModeRequestBody
.
class
);
final
String
topic
=
requestBody
.
getTopic
();
if
(
topic
.
startsWith
(
MixAll
.
RETRY_GROUP_TOPIC_PREFIX
))
{
response
.
setCode
(
ResponseCode
.
NO_PERMISSION
);
response
.
setRemark
(
"retry topic is not allowed to set mode"
);
return
response
;
}
final
String
consumerGroup
=
requestBody
.
getConsumerGroup
();
this
.
messageRequestModeManager
.
setMessageRequestMode
(
topic
,
consumerGroup
,
requestBody
);
this
.
messageRequestModeManager
.
persist
();
response
.
setCode
(
ResponseCode
.
SUCCESS
);
response
.
setRemark
(
null
);
return
response
;
}
}
broker/src/main/java/org/apache/rocketmq/broker/util/MsgUtil.java
0 → 100644
浏览文件 @
ced6b023
/*
* 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.util
;
import
org.apache.rocketmq.broker.BrokerController
;
import
org.apache.rocketmq.common.message.Message
;
import
org.apache.rocketmq.common.message.MessageExt
;
public
final
class
MsgUtil
{
private
MsgUtil
()
{
}
public
static
void
setMessageDeliverTime
(
BrokerController
brokerController
,
Message
msgInner
,
long
timeMillis
)
{
msgInner
.
setDelayTimeLevel
(
brokerController
.
getMessageStore
().
getScheduleMessageService
().
computeDelayLevel
(
timeMillis
));
}
public
static
long
getMessageDeliverTime
(
BrokerController
brokerController
,
MessageExt
msgInner
)
{
return
brokerController
.
getMessageStore
().
getScheduleMessageService
().
computeDeliverTimestamp
(
msgInner
.
getDelayTimeLevel
(),
msgInner
.
getStoreTimestamp
());
}
}
broker/src/test/java/org/apache/rocketmq/broker/processor/AckMessageProcessorTest.java
0 → 100644
浏览文件 @
ced6b023
/*
* 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.processor
;
import
io.netty.channel.Channel
;
import
io.netty.channel.ChannelHandlerContext
;
import
java.lang.reflect.Field
;
import
org.apache.rocketmq.broker.BrokerController
;
import
org.apache.rocketmq.broker.client.ClientChannelInfo
;
import
org.apache.rocketmq.broker.client.net.Broker2Client
;
import
org.apache.rocketmq.common.BrokerConfig
;
import
org.apache.rocketmq.common.TopicConfig
;
import
org.apache.rocketmq.common.message.MessageConst
;
import
org.apache.rocketmq.common.protocol.RequestCode
;
import
org.apache.rocketmq.common.protocol.ResponseCode
;
import
org.apache.rocketmq.common.protocol.header.AckMessageRequestHeader
;
import
org.apache.rocketmq.common.protocol.header.ExtraInfoUtil
;
import
org.apache.rocketmq.common.protocol.heartbeat.ConsumerData
;
import
org.apache.rocketmq.remoting.exception.RemotingCommandException
;
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.LanguageCode
;
import
org.apache.rocketmq.remoting.protocol.RemotingCommand
;
import
org.apache.rocketmq.store.AppendMessageResult
;
import
org.apache.rocketmq.store.AppendMessageStatus
;
import
org.apache.rocketmq.store.DefaultMessageStore
;
import
org.apache.rocketmq.store.MessageExtBrokerInner
;
import
org.apache.rocketmq.store.PutMessageResult
;
import
org.apache.rocketmq.store.PutMessageStatus
;
import
org.apache.rocketmq.store.config.MessageStoreConfig
;
import
org.apache.rocketmq.store.schedule.ScheduleMessageService
;
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
;
import
static
org
.
apache
.
rocketmq
.
broker
.
processor
.
PullMessageProcessorTest
.
createConsumerData
;
import
static
org
.
assertj
.
core
.
api
.
Assertions
.
assertThat
;
import
static
org
.
mockito
.
ArgumentMatchers
.
any
;
import
static
org
.
mockito
.
Mockito
.
mock
;
import
static
org
.
mockito
.
Mockito
.
when
;
@RunWith
(
MockitoJUnitRunner
.
class
)
public
class
AckMessageProcessorTest
{
private
AckMessageProcessor
ackMessageProcessor
;
@Spy
private
BrokerController
brokerController
=
new
BrokerController
(
new
BrokerConfig
(),
new
NettyServerConfig
(),
new
NettyClientConfig
(),
new
MessageStoreConfig
());
@Mock
private
ChannelHandlerContext
handlerContext
;
@Mock
private
DefaultMessageStore
messageStore
;
@Mock
private
Channel
channel
;
private
String
topic
=
"FooBar"
;
private
String
group
=
"FooBarGroup"
;
private
ClientChannelInfo
clientInfo
;
@Mock
private
Broker2Client
broker2Client
;
@Before
public
void
init
()
throws
IllegalAccessException
,
NoSuchFieldException
{
clientInfo
=
new
ClientChannelInfo
(
channel
,
"127.0.0.1"
,
LanguageCode
.
JAVA
,
0
);
brokerController
.
setMessageStore
(
messageStore
);
Field
field
=
BrokerController
.
class
.
getDeclaredField
(
"broker2Client"
);
field
.
setAccessible
(
true
);
field
.
set
(
brokerController
,
broker2Client
);
ScheduleMessageService
scheduleMessageService
=
new
ScheduleMessageService
(
messageStore
);
MessageStoreConfig
messageStoreConfig
=
new
MessageStoreConfig
();
messageStoreConfig
.
setMessageDelayLevel
(
"5s 10s"
);
when
(
messageStore
.
getMessageStoreConfig
()).
thenReturn
(
messageStoreConfig
);
scheduleMessageService
.
parseDelayLevel
();
when
(
messageStore
.
getScheduleMessageService
()).
thenReturn
(
scheduleMessageService
);
Channel
mockChannel
=
mock
(
Channel
.
class
);
when
(
handlerContext
.
channel
()).
thenReturn
(
mockChannel
);
brokerController
.
getTopicConfigManager
().
getTopicConfigTable
().
put
(
topic
,
new
TopicConfig
());
ConsumerData
consumerData
=
createConsumerData
(
group
,
topic
);
brokerController
.
getConsumerManager
().
registerConsumer
(
consumerData
.
getGroupName
(),
clientInfo
,
consumerData
.
getConsumeType
(),
consumerData
.
getMessageModel
(),
consumerData
.
getConsumeFromWhere
(),
consumerData
.
getSubscriptionDataSet
(),
false
);
ackMessageProcessor
=
new
AckMessageProcessor
(
brokerController
);
}
@Test
public
void
testProcessRequest_Success
()
throws
RemotingCommandException
,
InterruptedException
,
RemotingTimeoutException
,
RemotingSendRequestException
{
when
(
messageStore
.
putMessage
(
any
(
MessageExtBrokerInner
.
class
))).
thenReturn
(
new
PutMessageResult
(
PutMessageStatus
.
PUT_OK
,
new
AppendMessageResult
(
AppendMessageStatus
.
PUT_OK
)));
int
queueId
=
0
;
long
queueOffset
=
0
;
long
popTime
=
System
.
currentTimeMillis
()
-
1_000
;
long
invisibleTime
=
30_000
;
int
reviveQid
=
0
;
String
brokerName
=
"test_broker"
;
String
extraInfo
=
ExtraInfoUtil
.
buildExtraInfo
(
queueOffset
,
popTime
,
invisibleTime
,
reviveQid
,
topic
,
brokerName
,
queueId
)
+
MessageConst
.
KEY_SEPARATOR
+
queueOffset
;
AckMessageRequestHeader
requestHeader
=
new
AckMessageRequestHeader
();
requestHeader
.
setTopic
(
topic
);
requestHeader
.
setQueueId
(
0
);
requestHeader
.
setOffset
(
0L
);
requestHeader
.
setConsumerGroup
(
group
);
requestHeader
.
setExtraInfo
(
extraInfo
);
RemotingCommand
request
=
RemotingCommand
.
createRequestCommand
(
RequestCode
.
ACK_MESSAGE
,
requestHeader
);
request
.
makeCustomHeaderToNet
();
RemotingCommand
responseToReturn
=
ackMessageProcessor
.
processRequest
(
handlerContext
,
request
);
assertThat
(
responseToReturn
.
getCode
()).
isEqualTo
(
ResponseCode
.
SUCCESS
);
assertThat
(
responseToReturn
.
getOpaque
()).
isEqualTo
(
request
.
getOpaque
());
}
}
\ No newline at end of file
broker/src/test/java/org/apache/rocketmq/broker/processor/ChangeInvisibleTimeProcessorTest.java
0 → 100644
浏览文件 @
ced6b023
/*
* 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.processor
;
import
io.netty.channel.Channel
;
import
io.netty.channel.ChannelHandlerContext
;
import
java.lang.reflect.Field
;
import
org.apache.rocketmq.broker.BrokerController
;
import
org.apache.rocketmq.broker.client.ClientChannelInfo
;
import
org.apache.rocketmq.broker.client.net.Broker2Client
;
import
org.apache.rocketmq.common.BrokerConfig
;
import
org.apache.rocketmq.common.TopicConfig
;
import
org.apache.rocketmq.common.message.MessageConst
;
import
org.apache.rocketmq.common.protocol.RequestCode
;
import
org.apache.rocketmq.common.protocol.ResponseCode
;
import
org.apache.rocketmq.common.protocol.header.ChangeInvisibleTimeRequestHeader
;
import
org.apache.rocketmq.common.protocol.header.ExtraInfoUtil
;
import
org.apache.rocketmq.common.protocol.heartbeat.ConsumerData
;
import
org.apache.rocketmq.remoting.exception.RemotingCommandException
;
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.LanguageCode
;
import
org.apache.rocketmq.remoting.protocol.RemotingCommand
;
import
org.apache.rocketmq.store.AppendMessageResult
;
import
org.apache.rocketmq.store.AppendMessageStatus
;
import
org.apache.rocketmq.store.DefaultMessageStore
;
import
org.apache.rocketmq.store.MessageExtBrokerInner
;
import
org.apache.rocketmq.store.PutMessageResult
;
import
org.apache.rocketmq.store.PutMessageStatus
;
import
org.apache.rocketmq.store.config.MessageStoreConfig
;
import
org.apache.rocketmq.store.schedule.ScheduleMessageService
;
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
;
import
static
org
.
apache
.
rocketmq
.
broker
.
processor
.
PullMessageProcessorTest
.
createConsumerData
;
import
static
org
.
assertj
.
core
.
api
.
Assertions
.
assertThat
;
import
static
org
.
mockito
.
ArgumentMatchers
.
any
;
import
static
org
.
mockito
.
Mockito
.
mock
;
import
static
org
.
mockito
.
Mockito
.
when
;
@RunWith
(
MockitoJUnitRunner
.
class
)
public
class
ChangeInvisibleTimeProcessorTest
{
private
ChangeInvisibleTimeProcessor
changeInvisibleTimeProcessor
;
@Spy
private
BrokerController
brokerController
=
new
BrokerController
(
new
BrokerConfig
(),
new
NettyServerConfig
(),
new
NettyClientConfig
(),
new
MessageStoreConfig
());
@Mock
private
ChannelHandlerContext
handlerContext
;
@Mock
private
DefaultMessageStore
messageStore
;
@Mock
private
Channel
channel
;
private
String
topic
=
"FooBar"
;
private
String
group
=
"FooBarGroup"
;
private
ClientChannelInfo
clientInfo
;
@Mock
private
Broker2Client
broker2Client
;
@Before
public
void
init
()
throws
IllegalAccessException
,
NoSuchFieldException
{
brokerController
.
setMessageStore
(
messageStore
);
Field
field
=
BrokerController
.
class
.
getDeclaredField
(
"broker2Client"
);
field
.
setAccessible
(
true
);
field
.
set
(
brokerController
,
broker2Client
);
ScheduleMessageService
scheduleMessageService
=
new
ScheduleMessageService
(
messageStore
);
MessageStoreConfig
messageStoreConfig
=
new
MessageStoreConfig
();
messageStoreConfig
.
setMessageDelayLevel
(
"5s 10s"
);
when
(
messageStore
.
getMessageStoreConfig
()).
thenReturn
(
messageStoreConfig
);
scheduleMessageService
.
parseDelayLevel
();
when
(
messageStore
.
getScheduleMessageService
()).
thenReturn
(
scheduleMessageService
);
Channel
mockChannel
=
mock
(
Channel
.
class
);
when
(
handlerContext
.
channel
()).
thenReturn
(
mockChannel
);
brokerController
.
getTopicConfigManager
().
getTopicConfigTable
().
put
(
topic
,
new
TopicConfig
());
ConsumerData
consumerData
=
createConsumerData
(
group
,
topic
);
clientInfo
=
new
ClientChannelInfo
(
channel
,
"127.0.0.1"
,
LanguageCode
.
JAVA
,
0
);
brokerController
.
getConsumerManager
().
registerConsumer
(
consumerData
.
getGroupName
(),
clientInfo
,
consumerData
.
getConsumeType
(),
consumerData
.
getMessageModel
(),
consumerData
.
getConsumeFromWhere
(),
consumerData
.
getSubscriptionDataSet
(),
false
);
changeInvisibleTimeProcessor
=
new
ChangeInvisibleTimeProcessor
(
brokerController
);
}
@Test
public
void
testProcessRequest_Success
()
throws
RemotingCommandException
,
InterruptedException
,
RemotingTimeoutException
,
RemotingSendRequestException
{
when
(
messageStore
.
putMessage
(
any
(
MessageExtBrokerInner
.
class
))).
thenReturn
(
new
PutMessageResult
(
PutMessageStatus
.
PUT_OK
,
new
AppendMessageResult
(
AppendMessageStatus
.
PUT_OK
)));
int
queueId
=
0
;
long
queueOffset
=
0
;
long
popTime
=
System
.
currentTimeMillis
()
-
1_000
;
long
invisibleTime
=
30_000
;
int
reviveQid
=
0
;
String
brokerName
=
"test_broker"
;
String
extraInfo
=
ExtraInfoUtil
.
buildExtraInfo
(
queueOffset
,
popTime
,
invisibleTime
,
reviveQid
,
topic
,
brokerName
,
queueId
)
+
MessageConst
.
KEY_SEPARATOR
+
queueOffset
;
ChangeInvisibleTimeRequestHeader
requestHeader
=
new
ChangeInvisibleTimeRequestHeader
();
requestHeader
.
setTopic
(
topic
);
requestHeader
.
setQueueId
(
queueId
);
requestHeader
.
setOffset
(
queueOffset
);
requestHeader
.
setConsumerGroup
(
group
);
requestHeader
.
setExtraInfo
(
extraInfo
);
requestHeader
.
setInvisibleTime
(
invisibleTime
);
final
RemotingCommand
request
=
RemotingCommand
.
createRequestCommand
(
RequestCode
.
CHANGE_MESSAGE_INVISIBLETIME
,
requestHeader
);
request
.
makeCustomHeaderToNet
();
RemotingCommand
responseToReturn
=
changeInvisibleTimeProcessor
.
processRequest
(
handlerContext
,
request
);
assertThat
(
responseToReturn
.
getCode
()).
isEqualTo
(
ResponseCode
.
SUCCESS
);
assertThat
(
responseToReturn
.
getOpaque
()).
isEqualTo
(
request
.
getOpaque
());
}
}
\ No newline at end of file
broker/src/test/java/org/apache/rocketmq/broker/processor/PopBufferMergeServiceTest.java
0 → 100644
浏览文件 @
ced6b023
package
org.apache.rocketmq.broker.processor
;
import
io.netty.channel.Channel
;
import
io.netty.channel.ChannelHandlerContext
;
import
org.apache.commons.lang3.reflect.FieldUtils
;
import
org.apache.rocketmq.broker.BrokerController
;
import
org.apache.rocketmq.broker.client.ClientChannelInfo
;
import
org.apache.rocketmq.common.BrokerConfig
;
import
org.apache.rocketmq.common.TopicConfig
;
import
org.apache.rocketmq.common.protocol.heartbeat.ConsumerData
;
import
org.apache.rocketmq.remoting.netty.NettyClientConfig
;
import
org.apache.rocketmq.remoting.netty.NettyServerConfig
;
import
org.apache.rocketmq.store.DefaultMessageStore
;
import
org.apache.rocketmq.store.config.MessageStoreConfig
;
import
org.apache.rocketmq.store.pop.AckMsg
;
import
org.apache.rocketmq.store.pop.PopCheckPoint
;
import
org.apache.rocketmq.store.schedule.ScheduleMessageService
;
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
;
import
static
org
.
apache
.
rocketmq
.
broker
.
processor
.
PullMessageProcessorTest
.
createConsumerData
;
import
static
org
.
assertj
.
core
.
api
.
Assertions
.
assertThat
;
import
static
org
.
mockito
.
Mockito
.
mock
;
import
static
org
.
mockito
.
Mockito
.
when
;
@RunWith
(
MockitoJUnitRunner
.
class
)
public
class
PopBufferMergeServiceTest
{
@Spy
private
BrokerController
brokerController
=
new
BrokerController
(
new
BrokerConfig
(),
new
NettyServerConfig
(),
new
NettyClientConfig
(),
new
MessageStoreConfig
());
@Mock
private
PopMessageProcessor
popMessageProcessor
;
@Mock
private
ChannelHandlerContext
handlerContext
;
@Mock
private
DefaultMessageStore
messageStore
;
private
ScheduleMessageService
scheduleMessageService
;
private
ClientChannelInfo
clientChannelInfo
;
private
String
group
=
"FooBarGroup"
;
private
String
topic
=
"FooBar"
;
@Before
public
void
init
()
throws
Exception
{
FieldUtils
.
writeField
(
brokerController
.
getBrokerConfig
(),
"enablePopBufferMerge"
,
true
,
true
);
brokerController
.
setMessageStore
(
messageStore
);
popMessageProcessor
=
new
PopMessageProcessor
(
brokerController
);
scheduleMessageService
=
new
ScheduleMessageService
(
messageStore
);
MessageStoreConfig
messageStoreConfig
=
new
MessageStoreConfig
();
messageStoreConfig
.
setMessageDelayLevel
(
"5s 10s"
);
when
(
messageStore
.
getMessageStoreConfig
()).
thenReturn
(
messageStoreConfig
);
scheduleMessageService
.
parseDelayLevel
();
Channel
mockChannel
=
mock
(
Channel
.
class
);
brokerController
.
getTopicConfigManager
().
getTopicConfigTable
().
put
(
topic
,
new
TopicConfig
());
clientChannelInfo
=
new
ClientChannelInfo
(
mockChannel
);
ConsumerData
consumerData
=
createConsumerData
(
group
,
topic
);
brokerController
.
getConsumerManager
().
registerConsumer
(
consumerData
.
getGroupName
(),
clientChannelInfo
,
consumerData
.
getConsumeType
(),
consumerData
.
getMessageModel
(),
consumerData
.
getConsumeFromWhere
(),
consumerData
.
getSubscriptionDataSet
(),
false
);
}
@Test
(
timeout
=
10_000
)
public
void
testBasic
()
throws
Exception
{
PopBufferMergeService
popBufferMergeService
=
new
PopBufferMergeService
(
brokerController
,
popMessageProcessor
);
popBufferMergeService
.
start
();
PopCheckPoint
ck
=
new
PopCheckPoint
();
ck
.
setBitMap
(
0
);
int
msgCnt
=
1
;
ck
.
setNum
((
byte
)
msgCnt
);
long
popTime
=
System
.
currentTimeMillis
()
-
1000
;
ck
.
setPopTime
(
popTime
);
int
invisibleTime
=
30_000
;
ck
.
setInvisibleTime
(
invisibleTime
);
int
offset
=
100
;
ck
.
setStartOffset
(
offset
);
ck
.
setCId
(
group
);
ck
.
setTopic
(
topic
);
int
queueId
=
0
;
ck
.
setQueueId
((
byte
)
queueId
);
int
reviveQid
=
0
;
long
nextBeginOffset
=
101L
;
long
ackOffset
=
offset
;
AckMsg
ackMsg
=
new
AckMsg
();
ackMsg
.
setAckOffset
(
ackOffset
);
ackMsg
.
setStartOffset
(
offset
);
ackMsg
.
setConsumerGroup
(
group
);
ackMsg
.
setTopic
(
topic
);
ackMsg
.
setQueueId
(
queueId
);
ackMsg
.
setPopTime
(
popTime
);
try
{
assertThat
(
popBufferMergeService
.
addCk
(
ck
,
reviveQid
,
ackOffset
,
nextBeginOffset
)).
isTrue
();
assertThat
(
popBufferMergeService
.
getLatestOffset
(
topic
,
group
,
queueId
)).
isEqualTo
(
nextBeginOffset
);
Thread
.
sleep
(
1000
);
// wait background threads of PopBufferMergeService run for some time
assertThat
(
popBufferMergeService
.
addAk
(
reviveQid
,
ackMsg
)).
isTrue
();
assertThat
(
popBufferMergeService
.
getLatestOffset
(
topic
,
group
,
queueId
)).
isEqualTo
(
nextBeginOffset
);
}
finally
{
popBufferMergeService
.
shutdown
(
true
);
}
}
}
\ No newline at end of file
broker/src/test/java/org/apache/rocketmq/broker/processor/PopMessageProcessorTest.java
0 → 100644
浏览文件 @
ced6b023
/*
* 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.processor
;
import
io.netty.channel.Channel
;
import
io.netty.channel.ChannelHandlerContext
;
import
java.net.InetSocketAddress
;
import
java.nio.ByteBuffer
;
import
org.apache.rocketmq.broker.BrokerController
;
import
org.apache.rocketmq.broker.client.ClientChannelInfo
;
import
org.apache.rocketmq.common.BrokerConfig
;
import
org.apache.rocketmq.common.TopicConfig
;
import
org.apache.rocketmq.common.constant.ConsumeInitMode
;
import
org.apache.rocketmq.common.message.MessageDecoder
;
import
org.apache.rocketmq.common.protocol.RequestCode
;
import
org.apache.rocketmq.common.protocol.ResponseCode
;
import
org.apache.rocketmq.common.protocol.header.PopMessageRequestHeader
;
import
org.apache.rocketmq.common.protocol.heartbeat.ConsumerData
;
import
org.apache.rocketmq.remoting.exception.RemotingCommandException
;
import
org.apache.rocketmq.remoting.netty.NettyClientConfig
;
import
org.apache.rocketmq.remoting.netty.NettyServerConfig
;
import
org.apache.rocketmq.remoting.protocol.RemotingCommand
;
import
org.apache.rocketmq.store.AppendMessageResult
;
import
org.apache.rocketmq.store.AppendMessageStatus
;
import
org.apache.rocketmq.store.DefaultMessageStore
;
import
org.apache.rocketmq.store.GetMessageResult
;
import
org.apache.rocketmq.store.GetMessageStatus
;
import
org.apache.rocketmq.store.MappedFile
;
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.schedule.ScheduleMessageService
;
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
;
import
static
org
.
apache
.
rocketmq
.
broker
.
processor
.
PullMessageProcessorTest
.
createConsumerData
;
import
static
org
.
assertj
.
core
.
api
.
Assertions
.
assertThat
;
import
static
org
.
mockito
.
ArgumentMatchers
.
any
;
import
static
org
.
mockito
.
ArgumentMatchers
.
anyInt
;
import
static
org
.
mockito
.
ArgumentMatchers
.
anyLong
;
import
static
org
.
mockito
.
ArgumentMatchers
.
anyString
;
import
static
org
.
mockito
.
Mockito
.
mock
;
import
static
org
.
mockito
.
Mockito
.
when
;
@RunWith
(
MockitoJUnitRunner
.
class
)
public
class
PopMessageProcessorTest
{
private
PopMessageProcessor
popMessageProcessor
;
@Spy
private
BrokerController
brokerController
=
new
BrokerController
(
new
BrokerConfig
(),
new
NettyServerConfig
(),
new
NettyClientConfig
(),
new
MessageStoreConfig
());
@Mock
private
ChannelHandlerContext
handlerContext
;
@Mock
private
DefaultMessageStore
messageStore
;
private
ScheduleMessageService
scheduleMessageService
;
private
ClientChannelInfo
clientChannelInfo
;
private
String
group
=
"FooBarGroup"
;
private
String
topic
=
"FooBar"
;
@Before
public
void
init
()
{
brokerController
.
setMessageStore
(
messageStore
);
popMessageProcessor
=
new
PopMessageProcessor
(
brokerController
);
scheduleMessageService
=
new
ScheduleMessageService
(
messageStore
);
MessageStoreConfig
messageStoreConfig
=
new
MessageStoreConfig
();
messageStoreConfig
.
setMessageDelayLevel
(
"5s 10s"
);
when
(
messageStore
.
getMessageStoreConfig
()).
thenReturn
(
messageStoreConfig
);
scheduleMessageService
.
parseDelayLevel
();
when
(
messageStore
.
getScheduleMessageService
()).
thenReturn
(
scheduleMessageService
);
when
(
messageStore
.
putMessage
(
any
())).
thenReturn
(
new
PutMessageResult
(
PutMessageStatus
.
PUT_OK
,
new
AppendMessageResult
(
AppendMessageStatus
.
PUT_OK
)));
Channel
mockChannel
=
mock
(
Channel
.
class
);
when
(
mockChannel
.
remoteAddress
()).
thenReturn
(
new
InetSocketAddress
(
1024
));
when
(
handlerContext
.
channel
()).
thenReturn
(
mockChannel
);
brokerController
.
getTopicConfigManager
().
getTopicConfigTable
().
put
(
topic
,
new
TopicConfig
());
clientChannelInfo
=
new
ClientChannelInfo
(
mockChannel
);
ConsumerData
consumerData
=
createConsumerData
(
group
,
topic
);
brokerController
.
getConsumerManager
().
registerConsumer
(
consumerData
.
getGroupName
(),
clientChannelInfo
,
consumerData
.
getConsumeType
(),
consumerData
.
getMessageModel
(),
consumerData
.
getConsumeFromWhere
(),
consumerData
.
getSubscriptionDataSet
(),
false
);
}
@Test
public
void
testProcessRequest_TopicNotExist
()
throws
RemotingCommandException
{
brokerController
.
getTopicConfigManager
().
getTopicConfigTable
().
remove
(
topic
);
final
RemotingCommand
request
=
createPopMsgCommand
();
RemotingCommand
response
=
popMessageProcessor
.
processRequest
(
handlerContext
,
request
);
assertThat
(
response
).
isNotNull
();
assertThat
(
response
.
getCode
()).
isEqualTo
(
ResponseCode
.
TOPIC_NOT_EXIST
);
assertThat
(
response
.
getRemark
()).
contains
(
"topic["
+
topic
+
"] not exist"
);
}
@Test
public
void
testProcessRequest_SubNotExist
()
throws
RemotingCommandException
{
brokerController
.
getConsumerManager
().
unregisterConsumer
(
group
,
clientChannelInfo
,
false
);
final
RemotingCommand
request
=
createPopMsgCommand
();
RemotingCommand
response
=
popMessageProcessor
.
processRequest
(
handlerContext
,
request
);
assertThat
(
response
).
isNotNull
();
assertThat
(
response
.
getCode
()).
isEqualTo
(
ResponseCode
.
SUBSCRIPTION_NOT_EXIST
);
assertThat
(
response
.
getRemark
()).
contains
(
"consumer's group info not exist"
);
}
@Test
public
void
testProcessRequest_Found
()
throws
RemotingCommandException
{
GetMessageResult
getMessageResult
=
createGetMessageResult
(
1
);
when
(
messageStore
.
getMessage
(
anyString
(),
anyString
(),
anyInt
(),
anyLong
(),
anyInt
(),
any
())).
thenReturn
(
getMessageResult
);
final
RemotingCommand
request
=
createPopMsgCommand
();
RemotingCommand
response
=
popMessageProcessor
.
processRequest
(
handlerContext
,
request
);
assertThat
(
response
).
isNotNull
();
assertThat
(
response
.
getCode
()).
isEqualTo
(
ResponseCode
.
SUCCESS
);
}
@Test
public
void
testProcessRequest_MsgWasRemoving
()
throws
RemotingCommandException
{
GetMessageResult
getMessageResult
=
createGetMessageResult
(
1
);
getMessageResult
.
setStatus
(
GetMessageStatus
.
MESSAGE_WAS_REMOVING
);
when
(
messageStore
.
getMessage
(
anyString
(),
anyString
(),
anyInt
(),
anyLong
(),
anyInt
(),
any
())).
thenReturn
(
getMessageResult
);
final
RemotingCommand
request
=
createPopMsgCommand
();
RemotingCommand
response
=
popMessageProcessor
.
processRequest
(
handlerContext
,
request
);
assertThat
(
response
).
isNotNull
();
assertThat
(
response
.
getCode
()).
isEqualTo
(
ResponseCode
.
SUCCESS
);
}
@Test
public
void
testProcessRequest_NoMsgInQueue
()
throws
RemotingCommandException
{
GetMessageResult
getMessageResult
=
createGetMessageResult
(
0
);
getMessageResult
.
setStatus
(
GetMessageStatus
.
NO_MESSAGE_IN_QUEUE
);
when
(
messageStore
.
getMessage
(
anyString
(),
anyString
(),
anyInt
(),
anyLong
(),
anyInt
(),
any
())).
thenReturn
(
getMessageResult
);
final
RemotingCommand
request
=
createPopMsgCommand
();
RemotingCommand
response
=
popMessageProcessor
.
processRequest
(
handlerContext
,
request
);
assertThat
(
response
).
isNull
();
}
private
RemotingCommand
createPopMsgCommand
()
{
PopMessageRequestHeader
requestHeader
=
new
PopMessageRequestHeader
();
requestHeader
.
setConsumerGroup
(
group
);
requestHeader
.
setMaxMsgNums
(
30
);
requestHeader
.
setQueueId
(-
1
);
requestHeader
.
setTopic
(
topic
);
requestHeader
.
setInvisibleTime
(
10_000
);
requestHeader
.
setInitMode
(
ConsumeInitMode
.
MAX
);
requestHeader
.
setOrder
(
false
);
requestHeader
.
setPollTime
(
15_000
);
requestHeader
.
setBornTime
(
System
.
currentTimeMillis
());
RemotingCommand
request
=
RemotingCommand
.
createRequestCommand
(
RequestCode
.
POP_MESSAGE
,
requestHeader
);
request
.
makeCustomHeaderToNet
();
return
request
;
}
private
GetMessageResult
createGetMessageResult
(
int
msgCnt
)
{
GetMessageResult
getMessageResult
=
new
GetMessageResult
();
getMessageResult
.
setStatus
(
GetMessageStatus
.
FOUND
);
getMessageResult
.
setMinOffset
(
100
);
getMessageResult
.
setMaxOffset
(
1024
);
getMessageResult
.
setNextBeginOffset
(
516
);
for
(
int
i
=
0
;
i
<
msgCnt
;
i
++)
{
ByteBuffer
bb
=
ByteBuffer
.
allocate
(
64
);
bb
.
putLong
(
MessageDecoder
.
MESSAGE_STORE_TIMESTAMP_POSITION
,
System
.
currentTimeMillis
());
getMessageResult
.
addMessage
(
new
SelectMappedBufferResult
(
200
,
bb
,
64
,
new
MappedFile
()));
}
return
getMessageResult
;
}
}
\ No newline at end of file
broker/src/test/java/org/apache/rocketmq/broker/processor/QueryAssignmentProcessorTest.java
0 → 100644
浏览文件 @
ced6b023
/*
* 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.processor
;
import
com.google.common.collect.ImmutableSet
;
import
io.netty.channel.Channel
;
import
io.netty.channel.ChannelHandlerContext
;
import
org.apache.rocketmq.broker.BrokerController
;
import
org.apache.rocketmq.broker.client.ClientChannelInfo
;
import
org.apache.rocketmq.broker.loadbalance.AssignmentManager
;
import
org.apache.rocketmq.common.BrokerConfig
;
import
org.apache.rocketmq.common.MixAll
;
import
org.apache.rocketmq.common.TopicConfig
;
import
org.apache.rocketmq.common.message.MessageQueue
;
import
org.apache.rocketmq.common.message.MessageRequestMode
;
import
org.apache.rocketmq.common.protocol.RequestCode
;
import
org.apache.rocketmq.common.protocol.ResponseCode
;
import
org.apache.rocketmq.common.protocol.body.QueryAssignmentRequestBody
;
import
org.apache.rocketmq.common.protocol.body.QueryAssignmentResponseBody
;
import
org.apache.rocketmq.common.protocol.body.SetMessageRequestModeRequestBody
;
import
org.apache.rocketmq.common.protocol.heartbeat.ConsumerData
;
import
org.apache.rocketmq.common.protocol.heartbeat.MessageModel
;
import
org.apache.rocketmq.remoting.netty.NettyClientConfig
;
import
org.apache.rocketmq.remoting.netty.NettyServerConfig
;
import
org.apache.rocketmq.remoting.protocol.LanguageCode
;
import
org.apache.rocketmq.remoting.protocol.RemotingCommand
;
import
org.apache.rocketmq.store.MessageStore
;
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.Spy
;
import
org.mockito.junit.MockitoJUnitRunner
;
import
static
org
.
apache
.
rocketmq
.
broker
.
processor
.
PullMessageProcessorTest
.
createConsumerData
;
import
static
org
.
assertj
.
core
.
api
.
Assertions
.
assertThat
;
import
static
org
.
mockito
.
Mockito
.
doReturn
;
import
static
org
.
mockito
.
Mockito
.
when
;
@RunWith
(
MockitoJUnitRunner
.
class
)
public
class
QueryAssignmentProcessorTest
{
private
QueryAssignmentProcessor
queryAssignmentProcessor
;
@Spy
private
BrokerController
brokerController
=
new
BrokerController
(
new
BrokerConfig
(),
new
NettyServerConfig
(),
new
NettyClientConfig
(),
new
MessageStoreConfig
());
@Mock
private
AssignmentManager
assignmentManager
;
@Mock
private
ChannelHandlerContext
handlerContext
;
@Mock
private
MessageStore
messageStore
;
@Mock
private
Channel
channel
;
private
String
topic
=
"FooBar"
;
private
String
group
=
"FooBarGroup"
;
private
String
clientId
=
"127.0.0.1"
;
private
ClientChannelInfo
clientInfo
;
@Before
public
void
init
()
throws
IllegalAccessException
,
NoSuchFieldException
{
clientInfo
=
new
ClientChannelInfo
(
channel
,
"127.0.0.1"
,
LanguageCode
.
JAVA
,
0
);
brokerController
.
setMessageStore
(
messageStore
);
doReturn
(
assignmentManager
).
when
(
brokerController
).
getAssignmentManager
();
when
(
assignmentManager
.
getTopicSubscribeInfo
(
topic
)).
thenReturn
(
ImmutableSet
.
of
(
new
MessageQueue
(
topic
,
"broker-1"
,
0
),
new
MessageQueue
(
topic
,
"broker-2"
,
1
)));
queryAssignmentProcessor
=
new
QueryAssignmentProcessor
(
brokerController
);
brokerController
.
getTopicConfigManager
().
getTopicConfigTable
().
put
(
topic
,
new
TopicConfig
());
ConsumerData
consumerData
=
createConsumerData
(
group
,
topic
);
brokerController
.
getConsumerManager
().
registerConsumer
(
consumerData
.
getGroupName
(),
clientInfo
,
consumerData
.
getConsumeType
(),
consumerData
.
getMessageModel
(),
consumerData
.
getConsumeFromWhere
(),
consumerData
.
getSubscriptionDataSet
(),
false
);
}
@Test
public
void
testQueryAssignment
()
throws
Exception
{
brokerController
.
getProducerManager
().
registerProducer
(
group
,
clientInfo
);
final
RemotingCommand
request
=
createQueryAssignmentRequest
();
RemotingCommand
responseToReturn
=
queryAssignmentProcessor
.
processRequest
(
handlerContext
,
request
);
assertThat
(
responseToReturn
.
getCode
()).
isEqualTo
(
ResponseCode
.
SUCCESS
);
assertThat
(
responseToReturn
.
getBody
()).
isNotNull
();
QueryAssignmentResponseBody
responseBody
=
QueryAssignmentResponseBody
.
decode
(
responseToReturn
.
getBody
(),
QueryAssignmentResponseBody
.
class
);
assertThat
(
responseBody
.
getMessageQueueAssignments
()).
size
().
isEqualTo
(
2
);
}
@Test
public
void
testSetMessageRequestMode_Success
()
throws
Exception
{
brokerController
.
getProducerManager
().
registerProducer
(
group
,
clientInfo
);
final
RemotingCommand
request
=
createSetMessageRequestModeRequest
(
topic
);
RemotingCommand
responseToReturn
=
queryAssignmentProcessor
.
processRequest
(
handlerContext
,
request
);
assertThat
(
responseToReturn
.
getCode
()).
isEqualTo
(
ResponseCode
.
SUCCESS
);
}
@Test
public
void
testSetMessageRequestMode_RetryTopic
()
throws
Exception
{
brokerController
.
getProducerManager
().
registerProducer
(
group
,
clientInfo
);
final
RemotingCommand
request
=
createSetMessageRequestModeRequest
(
MixAll
.
RETRY_GROUP_TOPIC_PREFIX
+
topic
);
RemotingCommand
responseToReturn
=
queryAssignmentProcessor
.
processRequest
(
handlerContext
,
request
);
assertThat
(
responseToReturn
.
getCode
()).
isEqualTo
(
ResponseCode
.
NO_PERMISSION
);
}
private
RemotingCommand
createQueryAssignmentRequest
()
{
QueryAssignmentRequestBody
requestBody
=
new
QueryAssignmentRequestBody
();
requestBody
.
setTopic
(
topic
);
requestBody
.
setConsumerGroup
(
group
);
requestBody
.
setClientId
(
clientId
);
requestBody
.
setMessageModel
(
MessageModel
.
CLUSTERING
);
requestBody
.
setStrategyName
(
"AVG"
);
RemotingCommand
request
=
RemotingCommand
.
createRequestCommand
(
RequestCode
.
QUERY_ASSIGNMENT
,
null
);
request
.
setBody
(
requestBody
.
encode
());
return
request
;
}
private
RemotingCommand
createSetMessageRequestModeRequest
(
String
topic
)
{
RemotingCommand
request
=
RemotingCommand
.
createRequestCommand
(
RequestCode
.
SET_MESSAGE_REQUEST_MODE
,
null
);
SetMessageRequestModeRequestBody
requestBody
=
new
SetMessageRequestModeRequestBody
();
requestBody
.
setTopic
(
topic
);
requestBody
.
setConsumerGroup
(
group
);
requestBody
.
setMode
(
MessageRequestMode
.
POP
);
requestBody
.
setPopShareQueueNum
(
0
);
request
.
setBody
(
requestBody
.
encode
());
return
request
;
}
private
RemotingCommand
createResponse
(
int
code
,
RemotingCommand
request
)
{
RemotingCommand
response
=
RemotingCommand
.
createResponseCommand
(
null
);
response
.
setCode
(
code
);
response
.
setOpaque
(
request
.
getOpaque
());
return
response
;
}
}
\ No newline at end of file
client/src/main/java/org/apache/rocketmq/client/consumer/rebalance/AllocateMessageQueueAveragely.java
浏览文件 @
ced6b023
...
...
@@ -20,14 +20,22 @@ import java.util.ArrayList;
import
java.util.List
;
import
org.apache.rocketmq.client.consumer.AllocateMessageQueueStrategy
;
import
org.apache.rocketmq.client.log.ClientLogger
;
import
org.apache.rocketmq.logging.InternalLogger
;
import
org.apache.rocketmq.common.message.MessageQueue
;
import
org.apache.rocketmq.logging.InternalLogger
;
/**
* Average Hashing queue algorithm
*/
public
class
AllocateMessageQueueAveragely
implements
AllocateMessageQueueStrategy
{
private
final
InternalLogger
log
=
ClientLogger
.
getLog
();
private
InternalLogger
log
;
public
AllocateMessageQueueAveragely
()
{
log
=
ClientLogger
.
getLog
();
}
public
AllocateMessageQueueAveragely
(
InternalLogger
log
)
{
this
.
log
=
log
;
}
@Override
public
List
<
MessageQueue
>
allocate
(
String
consumerGroup
,
String
currentCID
,
List
<
MessageQueue
>
mqAll
,
...
...
client/src/main/java/org/apache/rocketmq/client/consumer/rebalance/AllocateMessageQueueAveragelyByCircle.java
浏览文件 @
ced6b023
...
...
@@ -20,14 +20,22 @@ import java.util.ArrayList;
import
java.util.List
;
import
org.apache.rocketmq.client.consumer.AllocateMessageQueueStrategy
;
import
org.apache.rocketmq.client.log.ClientLogger
;
import
org.apache.rocketmq.logging.InternalLogger
;
import
org.apache.rocketmq.common.message.MessageQueue
;
import
org.apache.rocketmq.logging.InternalLogger
;
/**
* Cycle average Hashing queue algorithm
*/
public
class
AllocateMessageQueueAveragelyByCircle
implements
AllocateMessageQueueStrategy
{
private
final
InternalLogger
log
=
ClientLogger
.
getLog
();
private
InternalLogger
log
;
public
AllocateMessageQueueAveragelyByCircle
()
{
log
=
ClientLogger
.
getLog
();
}
public
AllocateMessageQueueAveragelyByCircle
(
InternalLogger
log
)
{
this
.
log
=
log
;
}
@Override
public
List
<
MessageQueue
>
allocate
(
String
consumerGroup
,
String
currentCID
,
List
<
MessageQueue
>
mqAll
,
...
...
distribution/conf/logback_broker.xml
浏览文件 @
ced6b023
...
...
@@ -257,6 +257,30 @@
</triggeringPolicy>
</appender>
<appender
name=
"RocketmqPopAppender_inner"
class=
"ch.qos.logback.core.rolling.RollingFileAppender"
>
<file>
${user.home}/logs/rocketmqlogs/pop.log
</file>
<append>
true
</append>
<rollingPolicy
class=
"ch.qos.logback.core.rolling.FixedWindowRollingPolicy"
>
<fileNamePattern>
${user.home}/logs/rocketmqlogs/otherdays/pop.%i.log
</fileNamePattern>
<minIndex>
1
</minIndex>
<maxIndex>
20
</maxIndex>
</rollingPolicy>
<triggeringPolicy
class=
"ch.qos.logback.core.rolling.SizeBasedTriggeringPolicy"
>
<maxFileSize>
128MB
</maxFileSize>
</triggeringPolicy>
<encoder>
<pattern>
%d{yyy-MM-dd HH:mm:ss,GMT+8} %p %t - %m%n
</pattern>
<charset
class=
"java.nio.charset.Charset"
>
UTF-8
</charset>
</encoder>
</appender>
<appender
name=
"RocketmqPopAppender"
class=
"ch.qos.logback.classic.AsyncAppender"
>
<appender-ref
ref=
"RocketmqPopAppender_inner"
/>
</appender>
<appender
name=
"STDOUT"
class=
"ch.qos.logback.core.ConsoleAppender"
>
<append>
true
</append>
<encoder>
...
...
@@ -330,6 +354,11 @@
<appender-ref
ref=
"STDOUT"
/>
</logger>
<logger
name=
"RocketmqPop"
additivity=
"false"
>
<level
value=
"INFO"
/>
<appender-ref
ref=
"RocketmqPopAppender"
/>
</logger>
<root>
<level
value=
"INFO"
/>
<appender-ref
ref=
"DefaultAppender"
/>
...
...
pom.xml
浏览文件 @
ced6b023
...
...
@@ -437,7 +437,7 @@
<dependency>
<groupId>
org.mockito
</groupId>
<artifactId>
mockito-core
</artifactId>
<version>
2.2
3.0
</version>
<version>
2.2
8.2
</version>
<scope>
test
</scope>
</dependency>
<dependency>
...
...
@@ -571,6 +571,11 @@
<artifactId>
guava
</artifactId>
<version>
19.0
</version>
</dependency>
<dependency>
<groupId>
com.googlecode.concurrentlinkedhashmap
</groupId>
<artifactId>
concurrentlinkedhashmap-lru
</artifactId>
<version>
1.4.2
</version>
</dependency>
<dependency>
<groupId>
io.openmessaging
</groupId>
<artifactId>
openmessaging-api
</artifactId>
...
...
remoting/src/main/java/org/apache/rocketmq/remoting/common/RemotingHelper.java
浏览文件 @
ced6b023
...
...
@@ -172,11 +172,18 @@ public class RemotingHelper {
public
static
String
parseSocketAddressAddr
(
SocketAddress
socketAddress
)
{
if
(
socketAddress
!=
null
)
{
// Default toString of InetSocketAddress is "hostName/IP:port"
final
String
addr
=
socketAddress
.
toString
();
if
(
addr
.
length
()
>
0
)
{
if
(
addr
.
contains
(
"/"
))
{
String
[]
segments
=
addr
.
split
(
"/"
);
if
(
segments
.
length
>
1
)
{
return
segments
[
1
];
}
}
return
addr
.
substring
(
1
);
}
return
addr
;
}
return
""
;
}
...
...
编辑
预览
Markdown
is supported
0%
请重试
或
添加新附件
.
添加附件
取消
You are about to add
0
people
to the discussion. Proceed with caution.
先完成此消息的编辑!
取消
想要评论请
注册
或
登录