Skip to content
体验新版
项目
组织
正在加载...
登录
切换导航
打开侧边栏
Apache RocketMQ
Rocketmq
提交
99b42f24
R
Rocketmq
项目概览
Apache RocketMQ
/
Rocketmq
上一次同步 大约 3 年
通知
267
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看板
提交
99b42f24
编写于
5月 24, 2021
作者:
斜
斜阳
浏览文件
操作
浏览文件
下载
电子邮件补丁
差异文件
[ISSUE #2735] QueryMsgByUniqueKey tool should return all messages with same unique key
上级
7028af68
变更
4
隐藏空白更改
内联
并排
Showing
4 changed file
with
79 addition
and
95 deletion
+79
-95
client/src/main/java/org/apache/rocketmq/client/impl/MQAdminImpl.java
...ain/java/org/apache/rocketmq/client/impl/MQAdminImpl.java
+11
-5
tools/src/main/java/org/apache/rocketmq/tools/admin/DefaultMQAdminExt.java
...va/org/apache/rocketmq/tools/admin/DefaultMQAdminExt.java
+9
-3
tools/src/main/java/org/apache/rocketmq/tools/admin/DefaultMQAdminExtImpl.java
...rg/apache/rocketmq/tools/admin/DefaultMQAdminExtImpl.java
+9
-3
tools/src/main/java/org/apache/rocketmq/tools/command/message/QueryMsgByUniqueKeySubCommand.java
.../tools/command/message/QueryMsgByUniqueKeySubCommand.java
+50
-84
未找到文件。
client/src/main/java/org/apache/rocketmq/client/impl/MQAdminImpl.java
浏览文件 @
99b42f24
...
...
@@ -265,17 +265,23 @@ public class MQAdminImpl {
messageId
.
getOffset
(),
timeoutMillis
);
}
public
QueryResult
queryMessage
(
String
topic
,
String
key
,
int
maxNum
,
long
begin
,
long
end
)
throws
MQClientException
,
InterruptedException
{
public
QueryResult
queryMessage
(
String
topic
,
String
key
,
int
maxNum
,
long
begin
,
long
end
)
throws
MQClientException
,
InterruptedException
{
return
queryMessage
(
topic
,
key
,
maxNum
,
begin
,
end
,
false
);
}
public
QueryResult
queryMessageByUniqKey
(
String
topic
,
String
uniqKey
,
int
maxNum
,
long
begin
,
long
end
)
throws
MQClientException
,
InterruptedException
{
return
queryMessage
(
topic
,
uniqKey
,
maxNum
,
begin
,
end
,
true
);
}
public
MessageExt
queryMessageByUniqKey
(
String
topic
,
String
uniqKey
)
throws
InterruptedException
,
MQClientException
{
QueryResult
qr
=
this
.
queryMessage
(
topic
,
uniqKey
,
32
,
MessageClientIDSetter
.
getNearlyTimeFromID
(
uniqKey
).
getTime
()
-
1000
,
Long
.
MAX_VALUE
,
true
);
QueryResult
qr
=
queryMessageByUniqKey
(
topic
,
uniqKey
,
32
,
MessageClientIDSetter
.
getNearlyTimeFromID
(
uniqKey
).
getTime
()
-
1000
,
Long
.
MAX_VALUE
);
if
(
qr
!=
null
&&
qr
.
getMessageList
()
!=
null
&&
qr
.
getMessageList
().
size
()
>
0
)
{
return
qr
.
getMessageList
().
get
(
0
);
}
else
{
...
...
tools/src/main/java/org/apache/rocketmq/tools/admin/DefaultMQAdminExt.java
浏览文件 @
99b42f24
...
...
@@ -128,12 +128,18 @@ public class DefaultMQAdminExt extends ClientConfig implements MQAdminExt {
}
@Override
public
QueryResult
queryMessage
(
String
topic
,
String
key
,
int
maxNum
,
long
begin
,
long
end
)
throws
MQClientException
,
InterruptedException
{
public
QueryResult
queryMessage
(
String
topic
,
String
key
,
int
maxNum
,
long
begin
,
long
end
)
throws
MQClientException
,
InterruptedException
{
return
defaultMQAdminExtImpl
.
queryMessage
(
topic
,
key
,
maxNum
,
begin
,
end
);
}
public
QueryResult
queryMessageByUniqueKey
(
String
topic
,
String
key
,
int
maxNum
,
long
begin
,
long
end
)
throws
MQClientException
,
InterruptedException
{
return
defaultMQAdminExtImpl
.
queryMessageByUniqKey
(
topic
,
key
,
maxNum
,
begin
,
end
);
}
@Override
public
void
start
()
throws
MQClientException
{
defaultMQAdminExtImpl
.
start
();
...
...
tools/src/main/java/org/apache/rocketmq/tools/admin/DefaultMQAdminExtImpl.java
浏览文件 @
99b42f24
...
...
@@ -991,12 +991,18 @@ public class DefaultMQAdminExtImpl implements MQAdminExt, MQAdminExtInner {
}
@Override
public
QueryResult
queryMessage
(
String
topic
,
String
key
,
int
maxNum
,
long
begin
,
long
end
)
throws
MQClientException
,
InterruptedException
{
public
QueryResult
queryMessage
(
String
topic
,
String
key
,
int
maxNum
,
long
begin
,
long
end
)
throws
MQClientException
,
InterruptedException
{
return
this
.
mqClientInstance
.
getMQAdminImpl
().
queryMessage
(
topic
,
key
,
maxNum
,
begin
,
end
);
}
public
QueryResult
queryMessageByUniqKey
(
String
topic
,
String
key
,
int
maxNum
,
long
begin
,
long
end
)
throws
MQClientException
,
InterruptedException
{
return
this
.
mqClientInstance
.
getMQAdminImpl
().
queryMessageByUniqKey
(
topic
,
key
,
maxNum
,
begin
,
end
);
}
@Override
public
void
updateConsumeOffset
(
String
brokerAddr
,
String
consumeGroup
,
MessageQueue
mq
,
long
offset
)
throws
RemotingException
,
InterruptedException
,
MQBrokerException
{
...
...
tools/src/main/java/org/apache/rocketmq/tools/command/message/QueryMsgByUniqueKeySubCommand.java
浏览文件 @
99b42f24
...
...
@@ -24,6 +24,7 @@ import java.util.List;
import
org.apache.commons.cli.CommandLine
;
import
org.apache.commons.cli.Option
;
import
org.apache.commons.cli.Options
;
import
org.apache.rocketmq.client.QueryResult
;
import
org.apache.rocketmq.client.exception.MQBrokerException
;
import
org.apache.rocketmq.client.exception.MQClientException
;
import
org.apache.rocketmq.common.UtilAll
;
...
...
@@ -57,82 +58,40 @@ public class QueryMsgByUniqueKeySubCommand implements SubCommand {
}
}
public
static
void
queryById
(
final
DefaultMQAdminExt
admin
,
final
String
topic
,
final
String
msgId
)
throws
MQClientException
,
RemotingException
,
MQBrokerException
,
InterruptedException
,
IOException
{
MessageExt
msg
=
admin
.
viewMessage
(
topic
,
msgId
);
String
bodyTmpFilePath
=
createBodyFile
(
msg
);
System
.
out
.
printf
(
"%-20s %s%n"
,
"Topic:"
,
msg
.
getTopic
()
);
System
.
out
.
printf
(
"%-20s %s%n"
,
"Tags:"
,
"["
+
msg
.
getTags
()
+
"]"
);
System
.
out
.
printf
(
"%-20s %s%n"
,
"Keys:"
,
"["
+
msg
.
getKeys
()
+
"]"
);
System
.
out
.
printf
(
"%-20s %d%n"
,
"Queue ID:"
,
msg
.
getQueueId
()
);
System
.
out
.
printf
(
"%-20s %d%n"
,
"Queue Offset:"
,
msg
.
getQueueOffset
()
);
System
.
out
.
printf
(
"%-20s %d%n"
,
"CommitLog Offset:"
,
msg
.
getCommitLogOffset
()
);
System
.
out
.
printf
(
"%-20s %d%n"
,
"Reconsume Times:"
,
msg
.
getReconsumeTimes
()
);
System
.
out
.
printf
(
"%-20s %s%n"
,
"Born Timestamp:"
,
UtilAll
.
timeMillisToHumanString2
(
msg
.
getBornTimestamp
())
);
System
.
out
.
printf
(
"%-20s %s%n"
,
"Store Timestamp:"
,
UtilAll
.
timeMillisToHumanString2
(
msg
.
getStoreTimestamp
())
);
System
.
out
.
printf
(
"%-20s %s%n"
,
"Born Host:"
,
RemotingHelper
.
parseSocketAddressAddr
(
msg
.
getBornHost
())
);
System
.
out
.
printf
(
"%-20s %s%n"
,
"Store Host:"
,
RemotingHelper
.
parseSocketAddressAddr
(
msg
.
getStoreHost
())
);
System
.
out
.
printf
(
"%-20s %d%n"
,
"System Flag:"
,
msg
.
getSysFlag
()
);
System
.
out
.
printf
(
"%-20s %s%n"
,
"Properties:"
,
msg
.
getProperties
()
!=
null
?
msg
.
getProperties
().
toString
()
:
""
);
System
.
out
.
printf
(
"%-20s %s%n"
,
"Message Body Path:"
,
bodyTmpFilePath
);
public
static
void
queryById
(
final
DefaultMQAdminExt
admin
,
final
String
topic
,
final
String
msgId
,
final
boolean
showAll
)
throws
MQClientException
,
RemotingException
,
MQBrokerException
,
InterruptedException
,
IOException
{
QueryResult
queryResult
=
admin
.
queryMessageByUniqueKey
(
topic
,
msgId
,
32
,
0
,
Long
.
MAX_VALUE
);
assert
queryResult
!=
null
;
List
<
MessageExt
>
list
=
queryResult
.
getMessageList
();
list
.
sort
((
o1
,
o2
)
->
(
int
)
(
o1
.
getStoreTimestamp
()
-
o2
.
getStoreTimestamp
()));
for
(
int
i
=
0
;
i
<
(
showAll
?
list
.
size
()
:
1
);
i
++)
{
showMessage
(
admin
,
list
.
get
(
i
),
i
);
}
}
private
static
void
showMessage
(
final
DefaultMQAdminExt
admin
,
MessageExt
msg
,
int
index
)
throws
IOException
{
String
bodyTmpFilePath
=
createBodyFile
(
msg
,
index
);
final
String
strFormat
=
"%-20s %s%n"
;
final
String
intFormat
=
"%-20s %d%n"
;
System
.
out
.
printf
(
strFormat
,
"Topic:"
,
msg
.
getTopic
());
System
.
out
.
printf
(
strFormat
,
"Tags:"
,
"["
+
msg
.
getTags
()
+
"]"
);
System
.
out
.
printf
(
strFormat
,
"Keys:"
,
"["
+
msg
.
getKeys
()
+
"]"
);
System
.
out
.
printf
(
intFormat
,
"Queue ID:"
,
msg
.
getQueueId
());
System
.
out
.
printf
(
intFormat
,
"Queue Offset:"
,
msg
.
getQueueOffset
());
System
.
out
.
printf
(
intFormat
,
"CommitLog Offset:"
,
msg
.
getCommitLogOffset
());
System
.
out
.
printf
(
intFormat
,
"Reconsume Times:"
,
msg
.
getReconsumeTimes
());
System
.
out
.
printf
(
strFormat
,
"Born Timestamp:"
,
UtilAll
.
timeMillisToHumanString2
(
msg
.
getBornTimestamp
()));
System
.
out
.
printf
(
strFormat
,
"Store Timestamp:"
,
UtilAll
.
timeMillisToHumanString2
(
msg
.
getStoreTimestamp
()));
System
.
out
.
printf
(
strFormat
,
"Born Host:"
,
RemotingHelper
.
parseSocketAddressAddr
(
msg
.
getBornHost
()));
System
.
out
.
printf
(
strFormat
,
"Store Host:"
,
RemotingHelper
.
parseSocketAddressAddr
(
msg
.
getStoreHost
()));
System
.
out
.
printf
(
intFormat
,
"System Flag:"
,
msg
.
getSysFlag
());
System
.
out
.
printf
(
strFormat
,
"Properties:"
,
msg
.
getProperties
()
!=
null
?
msg
.
getProperties
().
toString
()
:
""
);
System
.
out
.
printf
(
strFormat
,
"Message Body Path:"
,
bodyTmpFilePath
);
try
{
List
<
MessageTrack
>
mtdList
=
admin
.
messageTrackDetail
(
msg
);
...
...
@@ -149,18 +108,21 @@ public class QueryMsgByUniqueKeySubCommand implements SubCommand {
}
}
private
static
String
createBodyFile
(
MessageExt
msg
)
throws
IOException
{
private
static
String
createBodyFile
(
MessageExt
msg
,
int
index
)
throws
IOException
{
DataOutputStream
dos
=
null
;
try
{
String
bodyTmpFilePath
=
"/tmp/rocketmq/msgbodys"
;
File
file
=
new
File
(
bodyTmpFilePath
);
String
Buffer
bodyTmpFilePath
=
new
StringBuffer
(
"/tmp/rocketmq/msgbodys"
)
;
File
file
=
new
File
(
bodyTmpFilePath
.
toString
()
);
if
(!
file
.
exists
())
{
file
.
mkdirs
();
}
bodyTmpFilePath
=
bodyTmpFilePath
+
"/"
+
msg
.
getMsgId
();
dos
=
new
DataOutputStream
(
new
FileOutputStream
(
bodyTmpFilePath
));
bodyTmpFilePath
.
append
(
"/"
).
append
(
msg
.
getMsgId
());
if
(
index
>
0
)
{
bodyTmpFilePath
.
append
(
"_"
+
index
);
}
dos
=
new
DataOutputStream
(
new
FileOutputStream
(
bodyTmpFilePath
.
toString
()));
dos
.
write
(
msg
.
getBody
());
return
bodyTmpFilePath
;
return
bodyTmpFilePath
.
toString
()
;
}
finally
{
if
(
dos
!=
null
)
dos
.
close
();
...
...
@@ -195,6 +157,10 @@ public class QueryMsgByUniqueKeySubCommand implements SubCommand {
opt
.
setRequired
(
true
);
options
.
addOption
(
opt
);
opt
=
new
Option
(
"a"
,
"showAll"
,
false
,
"Print all message, the limit is 32"
);
opt
.
setRequired
(
false
);
options
.
addOption
(
opt
);
return
options
;
}
...
...
@@ -202,11 +168,11 @@ public class QueryMsgByUniqueKeySubCommand implements SubCommand {
public
void
execute
(
CommandLine
commandLine
,
Options
options
,
RPCHook
rpcHook
)
throws
SubCommandException
{
try
{
defaultMQAdminExt
=
createMQAdminExt
(
rpcHook
);
final
String
msgId
=
commandLine
.
getOptionValue
(
'i'
).
trim
();
final
String
topic
=
commandLine
.
getOptionValue
(
't'
).
trim
();
final
boolean
showAll
=
commandLine
.
hasOption
(
'a'
);
if
(
commandLine
.
hasOption
(
'g'
)
&&
commandLine
.
hasOption
(
'd'
))
{
final
String
consumerGroup
=
commandLine
.
getOptionValue
(
'g'
).
trim
();
final
String
clientId
=
commandLine
.
getOptionValue
(
'd'
).
trim
();
...
...
@@ -214,7 +180,7 @@ public class QueryMsgByUniqueKeySubCommand implements SubCommand {
defaultMQAdminExt
.
consumeMessageDirectly
(
consumerGroup
,
clientId
,
topic
,
msgId
);
System
.
out
.
printf
(
"%s"
,
result
);
}
else
{
queryById
(
defaultMQAdminExt
,
topic
,
msgId
);
queryById
(
defaultMQAdminExt
,
topic
,
msgId
,
showAll
);
}
}
catch
(
Exception
e
)
{
throw
new
SubCommandException
(
this
.
getClass
().
getSimpleName
()
+
" command failed"
,
e
);
...
...
编辑
预览
Markdown
is supported
0%
请重试
或
添加新附件
.
添加附件
取消
You are about to add
0
people
to the discussion. Proceed with caution.
先完成此消息的编辑!
取消
想要评论请
注册
或
登录