Skip to content
体验新版
项目
组织
正在加载...
登录
切换导航
打开侧边栏
小五666\n哈哈
Rocketmq
提交
92f0e1f5
R
Rocketmq
项目概览
小五666\n哈哈
/
Rocketmq
与 Fork 源项目一致
Fork自
Apache RocketMQ / Rocketmq
通知
1
Star
0
Fork
0
代码
文件
提交
分支
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看板
体验新版 GitCode,发现更多精彩内容 >>
提交
92f0e1f5
编写于
5月 25, 2018
作者:
傅
傅冲
提交者:
Li Zhanhui
6月 22, 2018
浏览文件
操作
浏览文件
下载
电子邮件补丁
差异文件
Optimize broker topic route registration to relieve stress on Java GC
上级
9932819e
变更
6
显示空白变更内容
内联
并排
Showing
6 changed file
with
107 addition
and
52 deletion
+107
-52
broker/src/main/java/org/apache/rocketmq/broker/BrokerController.java
...ain/java/org/apache/rocketmq/broker/BrokerController.java
+47
-22
broker/src/main/java/org/apache/rocketmq/broker/out/BrokerOuterAPI.java
...n/java/org/apache/rocketmq/broker/out/BrokerOuterAPI.java
+20
-22
broker/src/main/java/org/apache/rocketmq/broker/processor/AdminBrokerProcessor.java
...pache/rocketmq/broker/processor/AdminBrokerProcessor.java
+4
-6
common/src/main/java/org/apache/rocketmq/common/BrokerConfig.java
...rc/main/java/org/apache/rocketmq/common/BrokerConfig.java
+0
-2
common/src/main/java/org/apache/rocketmq/common/protocol/header/namesrv/RegisterBrokerRequestHeader.java
.../protocol/header/namesrv/RegisterBrokerRequestHeader.java
+10
-0
namesrv/src/main/java/org/apache/rocketmq/namesrv/processor/DefaultRequestProcessor.java
...e/rocketmq/namesrv/processor/DefaultRequestProcessor.java
+26
-0
未找到文件。
broker/src/main/java/org/apache/rocketmq/broker/BrokerController.java
浏览文件 @
92f0e1f5
...
...
@@ -24,6 +24,7 @@ import java.util.List;
import
java.util.Map
;
import
java.util.concurrent.BlockingQueue
;
import
java.util.concurrent.ConcurrentHashMap
;
import
java.util.concurrent.ConcurrentMap
;
import
java.util.concurrent.ExecutorService
;
import
java.util.concurrent.Executors
;
import
java.util.concurrent.LinkedBlockingQueue
;
...
...
@@ -62,6 +63,7 @@ import org.apache.rocketmq.broker.subscription.SubscriptionGroupManager;
import
org.apache.rocketmq.broker.topic.TopicConfigManager
;
import
org.apache.rocketmq.common.BrokerConfig
;
import
org.apache.rocketmq.common.Configuration
;
import
org.apache.rocketmq.common.DataVersion
;
import
org.apache.rocketmq.common.ThreadFactoryImpl
;
import
org.apache.rocketmq.common.TopicConfig
;
import
org.apache.rocketmq.common.UtilAll
;
...
...
@@ -768,6 +770,24 @@ public class BrokerController {
}
}
public
synchronized
void
registerIncrementBrokerData
(
TopicConfig
topicConfig
,
DataVersion
dataVersion
)
{
TopicConfig
registerTopicConfig
=
topicConfig
;
if
(!
PermName
.
isWriteable
(
this
.
getBrokerConfig
().
getBrokerPermission
())
||
!
PermName
.
isReadable
(
this
.
getBrokerConfig
().
getBrokerPermission
()))
{
registerTopicConfig
=
new
TopicConfig
(
topicConfig
.
getTopicName
(),
topicConfig
.
getReadQueueNums
(),
topicConfig
.
getWriteQueueNums
(),
this
.
brokerConfig
.
getBrokerPermission
());
}
ConcurrentMap
<
String
,
TopicConfig
>
topicConfigTable
=
new
ConcurrentHashMap
<
String
,
TopicConfig
>();
topicConfigTable
.
put
(
topicConfig
.
getTopicName
(),
registerTopicConfig
);
TopicConfigSerializeWrapper
topicConfigSerializeWrapper
=
new
TopicConfigSerializeWrapper
();
topicConfigSerializeWrapper
.
setDataVersion
(
dataVersion
);
topicConfigSerializeWrapper
.
setTopicConfigTable
(
topicConfigTable
);
doRegisterBrokerAll
(
true
,
false
,
topicConfigSerializeWrapper
);
}
public
synchronized
void
registerBrokerAll
(
final
boolean
checkOrderConfig
,
boolean
oneway
,
boolean
forceRegister
)
{
TopicConfigSerializeWrapper
topicConfigWrapper
=
this
.
getTopicConfigManager
().
buildTopicConfigSerializeWrapper
();
...
...
@@ -788,6 +808,12 @@ public class BrokerController {
this
.
brokerConfig
.
getBrokerName
(),
this
.
brokerConfig
.
getBrokerId
(),
this
.
brokerConfig
.
getRegisterBrokerTimeoutMills
()))
{
doRegisterBrokerAll
(
checkOrderConfig
,
oneway
,
topicConfigWrapper
);
}
}
private
void
doRegisterBrokerAll
(
boolean
checkOrderConfig
,
boolean
oneway
,
TopicConfigSerializeWrapper
topicConfigWrapper
)
{
List
<
RegisterBrokerResult
>
registerBrokerResultList
=
this
.
brokerOuterAPI
.
registerBrokerAll
(
this
.
brokerConfig
.
getBrokerClusterName
(),
this
.
getBrokerAddr
(),
...
...
@@ -815,7 +841,6 @@ public class BrokerController {
}
}
}
}
private
boolean
needRegister
(
final
String
clusterName
,
final
String
brokerAddr
,
...
...
broker/src/main/java/org/apache/rocketmq/broker/out/BrokerOuterAPI.java
浏览文件 @
92f0e1f5
...
...
@@ -29,6 +29,7 @@ import org.apache.rocketmq.client.exception.MQBrokerException;
import
org.apache.rocketmq.common.DataVersion
;
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
;
...
...
@@ -125,14 +126,28 @@ public class BrokerOuterAPI {
final
List
<
RegisterBrokerResult
>
registerBrokerResultList
=
Lists
.
newArrayList
();
List
<
String
>
nameServerAddressList
=
this
.
remotingClient
.
getNameServerAddressList
();
if
(
nameServerAddressList
!=
null
&&
nameServerAddressList
.
size
()
>
0
)
{
final
RegisterBrokerRequestHeader
requestHeader
=
new
RegisterBrokerRequestHeader
();
requestHeader
.
setBrokerAddr
(
brokerAddr
);
requestHeader
.
setBrokerId
(
brokerId
);
requestHeader
.
setBrokerName
(
brokerName
);
requestHeader
.
setClusterName
(
clusterName
);
requestHeader
.
setHaServerAddr
(
haServerAddr
);
requestHeader
.
setCompressed
(
compressed
);
RegisterBrokerBody
requestBody
=
new
RegisterBrokerBody
();
requestBody
.
setTopicConfigSerializeWrapper
(
topicConfigWrapper
);
requestBody
.
setFilterServerList
(
filterServerList
);
final
byte
[]
body
=
requestBody
.
encode
(
compressed
);
final
int
bodyCrc32
=
UtilAll
.
crc32
(
body
);
requestHeader
.
setBodyCrc32
(
bodyCrc32
);
final
CountDownLatch
countDownLatch
=
new
CountDownLatch
(
nameServerAddressList
.
size
());
for
(
final
String
namesrvAddr
:
nameServerAddressList
)
{
brokerOuterExecutor
.
execute
(
new
Runnable
()
{
@Override
public
void
run
()
{
try
{
RegisterBrokerResult
result
=
registerBroker
(
namesrvAddr
,
clusterName
,
brokerAddr
,
brokerName
,
brokerId
,
haServerAddr
,
topicConfigWrapper
,
filterServerList
,
oneway
,
timeoutMills
,
compressed
);
RegisterBrokerResult
result
=
registerBroker
(
namesrvAddr
,
oneway
,
timeoutMills
,
requestHeader
,
body
);
if
(
result
!=
null
)
{
registerBrokerResultList
.
add
(
result
);
}
...
...
@@ -158,31 +173,14 @@ public class BrokerOuterAPI {
private
RegisterBrokerResult
registerBroker
(
final
String
namesrvAddr
,
final
String
clusterName
,
final
String
brokerAddr
,
final
String
brokerName
,
final
long
brokerId
,
final
String
haServerAddr
,
final
TopicConfigSerializeWrapper
topicConfigWrapper
,
final
List
<
String
>
filterServerList
,
final
boolean
oneway
,
final
int
timeoutMills
,
final
boolean
compressed
final
RegisterBrokerRequestHeader
requestHeader
,
final
byte
[]
body
)
throws
RemotingCommandException
,
MQBrokerException
,
RemotingConnectException
,
RemotingSendRequestException
,
RemotingTimeoutException
,
InterruptedException
{
RegisterBrokerRequestHeader
requestHeader
=
new
RegisterBrokerRequestHeader
();
requestHeader
.
setBrokerAddr
(
brokerAddr
);
requestHeader
.
setBrokerId
(
brokerId
);
requestHeader
.
setBrokerName
(
brokerName
);
requestHeader
.
setClusterName
(
clusterName
);
requestHeader
.
setHaServerAddr
(
haServerAddr
);
requestHeader
.
setCompressed
(
compressed
);
RemotingCommand
request
=
RemotingCommand
.
createRequestCommand
(
RequestCode
.
REGISTER_BROKER
,
requestHeader
);
RegisterBrokerBody
requestBody
=
new
RegisterBrokerBody
();
requestBody
.
setTopicConfigSerializeWrapper
(
topicConfigWrapper
);
requestBody
.
setFilterServerList
(
filterServerList
);
request
.
setBody
(
requestBody
.
encode
(
requestHeader
.
isCompressed
()));
request
.
setBody
(
body
);
if
(
oneway
)
{
try
{
...
...
broker/src/main/java/org/apache/rocketmq/broker/processor/AdminBrokerProcessor.java
浏览文件 @
92f0e1f5
...
...
@@ -212,7 +212,7 @@ public class AdminBrokerProcessor implements NettyRequestProcessor {
return
false
;
}
private
RemotingCommand
updateAndCreateTopic
(
ChannelHandlerContext
ctx
,
private
synchronized
RemotingCommand
updateAndCreateTopic
(
ChannelHandlerContext
ctx
,
RemotingCommand
request
)
throws
RemotingCommandException
{
final
RemotingCommand
response
=
RemotingCommand
.
createResponseCommand
(
null
);
final
CreateTopicRequestHeader
requestHeader
=
...
...
@@ -246,14 +246,12 @@ public class AdminBrokerProcessor implements NettyRequestProcessor {
this
.
brokerController
.
getTopicConfigManager
().
updateTopicConfig
(
topicConfig
);
if
(
brokerController
.
getBrokerConfig
().
getRegisterNameServerPeriod
()
==
0
)
{
this
.
brokerController
.
registerBrokerAll
(
false
,
true
,
true
);
}
this
.
brokerController
.
registerIncrementBrokerData
(
topicConfig
,
this
.
brokerController
.
getTopicConfigManager
().
getDataVersion
());
return
null
;
}
private
RemotingCommand
deleteTopic
(
ChannelHandlerContext
ctx
,
private
synchronized
RemotingCommand
deleteTopic
(
ChannelHandlerContext
ctx
,
RemotingCommand
request
)
throws
RemotingCommandException
{
final
RemotingCommand
response
=
RemotingCommand
.
createResponseCommand
(
null
);
DeleteTopicRequestHeader
requestHeader
=
...
...
@@ -299,7 +297,7 @@ public class AdminBrokerProcessor implements NettyRequestProcessor {
return
response
;
}
private
RemotingCommand
updateBrokerConfig
(
ChannelHandlerContext
ctx
,
RemotingCommand
request
)
{
private
synchronized
RemotingCommand
updateBrokerConfig
(
ChannelHandlerContext
ctx
,
RemotingCommand
request
)
{
final
RemotingCommand
response
=
RemotingCommand
.
createResponseCommand
(
null
);
log
.
info
(
"updateBrokerConfig called by {}"
,
RemotingHelper
.
parseChannelRemoteAddr
(
ctx
.
channel
()));
...
...
common/src/main/java/org/apache/rocketmq/common/BrokerConfig.java
浏览文件 @
92f0e1f5
...
...
@@ -141,8 +141,6 @@ public class BrokerConfig {
* This configurable item defines interval of topics registration of broker to name server. Allowing values are
* between 10, 000 and 60, 000 milliseconds.
*
* If set to 0, newly created topics will be immediately reported to name servers and interval of periodical
* registration defaults to 10, 000 in milliseconds.
*/
private
int
registerNameServerPeriod
=
1000
*
30
;
...
...
common/src/main/java/org/apache/rocketmq/common/protocol/header/namesrv/RegisterBrokerRequestHeader.java
浏览文件 @
92f0e1f5
...
...
@@ -38,6 +38,8 @@ public class RegisterBrokerRequestHeader implements CommandCustomHeader {
private
boolean
compressed
;
private
Integer
bodyCrc32
=
0
;
public
void
checkFields
()
throws
RemotingCommandException
{
}
...
...
@@ -88,4 +90,12 @@ public class RegisterBrokerRequestHeader implements CommandCustomHeader {
public
void
setCompressed
(
boolean
compressed
)
{
this
.
compressed
=
compressed
;
}
public
Integer
getBodyCrc32
()
{
return
bodyCrc32
;
}
public
void
setBodyCrc32
(
Integer
bodyCrc32
)
{
this
.
bodyCrc32
=
bodyCrc32
;
}
}
namesrv/src/main/java/org/apache/rocketmq/namesrv/processor/DefaultRequestProcessor.java
浏览文件 @
92f0e1f5
...
...
@@ -24,6 +24,7 @@ import org.apache.rocketmq.common.DataVersion;
import
org.apache.rocketmq.common.MQVersion
;
import
org.apache.rocketmq.common.MQVersion.Version
;
import
org.apache.rocketmq.common.MixAll
;
import
org.apache.rocketmq.common.UtilAll
;
import
org.apache.rocketmq.common.constant.LoggerName
;
import
org.apache.rocketmq.common.help.FAQUrl
;
import
org.apache.rocketmq.logging.InternalLogger
;
...
...
@@ -196,6 +197,12 @@ public class DefaultRequestProcessor implements NettyRequestProcessor {
final
RegisterBrokerRequestHeader
requestHeader
=
(
RegisterBrokerRequestHeader
)
request
.
decodeCommandCustomHeader
(
RegisterBrokerRequestHeader
.
class
);
if
(!
checksum
(
ctx
,
request
,
requestHeader
))
{
response
.
setCode
(
ResponseCode
.
SYSTEM_ERROR
);
response
.
setRemark
(
"crc32 not match"
);
return
response
;
}
RegisterBrokerBody
registerBrokerBody
=
new
RegisterBrokerBody
();
if
(
request
.
getBody
()
!=
null
)
{
...
...
@@ -230,6 +237,19 @@ public class DefaultRequestProcessor implements NettyRequestProcessor {
return
response
;
}
private
boolean
checksum
(
ChannelHandlerContext
ctx
,
RemotingCommand
request
,
RegisterBrokerRequestHeader
requestHeader
)
{
if
(
requestHeader
.
getBodyCrc32
()
!=
0
)
{
final
int
crc32
=
UtilAll
.
crc32
(
request
.
getBody
());
if
(
crc32
!=
requestHeader
.
getBodyCrc32
())
{
log
.
warn
(
String
.
format
(
"receive registerBroker request,crc32 not match,from %s"
,
RemotingHelper
.
parseChannelRemoteAddr
(
ctx
.
channel
())));
return
false
;
}
}
return
true
;
}
public
RemotingCommand
queryBrokerTopicConfig
(
ChannelHandlerContext
ctx
,
RemotingCommand
request
)
throws
RemotingCommandException
{
final
RemotingCommand
response
=
RemotingCommand
.
createResponseCommand
(
QueryDataVersionResponseHeader
.
class
);
...
...
@@ -261,6 +281,12 @@ public class DefaultRequestProcessor implements NettyRequestProcessor {
final
RegisterBrokerRequestHeader
requestHeader
=
(
RegisterBrokerRequestHeader
)
request
.
decodeCommandCustomHeader
(
RegisterBrokerRequestHeader
.
class
);
if
(!
checksum
(
ctx
,
request
,
requestHeader
))
{
response
.
setCode
(
ResponseCode
.
SYSTEM_ERROR
);
response
.
setRemark
(
"crc32 not match"
);
return
response
;
}
TopicConfigSerializeWrapper
topicConfigWrapper
;
if
(
request
.
getBody
()
!=
null
)
{
topicConfigWrapper
=
TopicConfigSerializeWrapper
.
decode
(
request
.
getBody
(),
TopicConfigSerializeWrapper
.
class
);
...
...
编辑
预览
Markdown
is supported
0%
请重试
或
添加新附件
.
添加附件
取消
You are about to add
0
people
to the discussion. Proceed with caution.
先完成此消息的编辑!
取消
想要评论请
注册
或
登录