mirror of
https://github.com/didi/KnowStreaming.git
synced 2026-01-04 11:52:07 +08:00
[Bugfix] 修复新建Topic后,立即查看Topic-Messages信息会提示Topic不存在的问题 (#697)
This commit is contained in:
@@ -10,14 +10,18 @@ import com.xiaojukeji.know.streaming.km.common.bean.entity.cluster.ClusterPhy;
|
|||||||
import com.xiaojukeji.know.streaming.km.common.bean.entity.param.topic.TopicCreateParam;
|
import com.xiaojukeji.know.streaming.km.common.bean.entity.param.topic.TopicCreateParam;
|
||||||
import com.xiaojukeji.know.streaming.km.common.bean.entity.param.topic.TopicParam;
|
import com.xiaojukeji.know.streaming.km.common.bean.entity.param.topic.TopicParam;
|
||||||
import com.xiaojukeji.know.streaming.km.common.bean.entity.param.topic.TopicPartitionExpandParam;
|
import com.xiaojukeji.know.streaming.km.common.bean.entity.param.topic.TopicPartitionExpandParam;
|
||||||
|
import com.xiaojukeji.know.streaming.km.common.bean.entity.partition.Partition;
|
||||||
import com.xiaojukeji.know.streaming.km.common.bean.entity.result.Result;
|
import com.xiaojukeji.know.streaming.km.common.bean.entity.result.Result;
|
||||||
import com.xiaojukeji.know.streaming.km.common.bean.entity.result.ResultStatus;
|
import com.xiaojukeji.know.streaming.km.common.bean.entity.result.ResultStatus;
|
||||||
import com.xiaojukeji.know.streaming.km.common.bean.entity.topic.Topic;
|
import com.xiaojukeji.know.streaming.km.common.bean.entity.topic.Topic;
|
||||||
import com.xiaojukeji.know.streaming.km.common.constant.MsgConstant;
|
import com.xiaojukeji.know.streaming.km.common.constant.MsgConstant;
|
||||||
|
import com.xiaojukeji.know.streaming.km.common.utils.BackoffUtils;
|
||||||
|
import com.xiaojukeji.know.streaming.km.common.utils.FutureUtil;
|
||||||
import com.xiaojukeji.know.streaming.km.common.utils.ValidateUtils;
|
import com.xiaojukeji.know.streaming.km.common.utils.ValidateUtils;
|
||||||
import com.xiaojukeji.know.streaming.km.common.utils.kafka.KafkaReplicaAssignUtil;
|
import com.xiaojukeji.know.streaming.km.common.utils.kafka.KafkaReplicaAssignUtil;
|
||||||
import com.xiaojukeji.know.streaming.km.core.service.broker.BrokerService;
|
import com.xiaojukeji.know.streaming.km.core.service.broker.BrokerService;
|
||||||
import com.xiaojukeji.know.streaming.km.core.service.cluster.ClusterPhyService;
|
import com.xiaojukeji.know.streaming.km.core.service.cluster.ClusterPhyService;
|
||||||
|
import com.xiaojukeji.know.streaming.km.core.service.partition.PartitionService;
|
||||||
import com.xiaojukeji.know.streaming.km.core.service.topic.OpTopicService;
|
import com.xiaojukeji.know.streaming.km.core.service.topic.OpTopicService;
|
||||||
import com.xiaojukeji.know.streaming.km.core.service.topic.TopicService;
|
import com.xiaojukeji.know.streaming.km.core.service.topic.TopicService;
|
||||||
import kafka.admin.AdminUtils;
|
import kafka.admin.AdminUtils;
|
||||||
@@ -52,6 +56,9 @@ public class OpTopicManagerImpl implements OpTopicManager {
|
|||||||
@Autowired
|
@Autowired
|
||||||
private ClusterPhyService clusterPhyService;
|
private ClusterPhyService clusterPhyService;
|
||||||
|
|
||||||
|
@Autowired
|
||||||
|
private PartitionService partitionService;
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
public Result<Void> createTopic(TopicCreateDTO dto, String operator) {
|
public Result<Void> createTopic(TopicCreateDTO dto, String operator) {
|
||||||
log.info("method=createTopic||param={}||operator={}.", dto, operator);
|
log.info("method=createTopic||param={}||operator={}.", dto, operator);
|
||||||
@@ -80,7 +87,7 @@ public class OpTopicManagerImpl implements OpTopicManager {
|
|||||||
);
|
);
|
||||||
|
|
||||||
// 创建Topic
|
// 创建Topic
|
||||||
return opTopicService.createTopic(
|
Result<Void> createTopicRes = opTopicService.createTopic(
|
||||||
new TopicCreateParam(
|
new TopicCreateParam(
|
||||||
dto.getClusterId(),
|
dto.getClusterId(),
|
||||||
dto.getTopicName(),
|
dto.getTopicName(),
|
||||||
@@ -90,6 +97,21 @@ public class OpTopicManagerImpl implements OpTopicManager {
|
|||||||
),
|
),
|
||||||
operator
|
operator
|
||||||
);
|
);
|
||||||
|
if (createTopicRes.successful()){
|
||||||
|
try{
|
||||||
|
FutureUtil.quickStartupFutureUtil.submitTask(() -> {
|
||||||
|
BackoffUtils.backoff(3000);
|
||||||
|
Result<List<Partition>> partitionsResult = partitionService.listPartitionsFromKafka(clusterPhy, dto.getTopicName());
|
||||||
|
if (partitionsResult.successful()){
|
||||||
|
partitionService.updatePartitions(clusterPhy.getId(), dto.getTopicName(), partitionsResult.getData(), new ArrayList<>());
|
||||||
|
}
|
||||||
|
});
|
||||||
|
}catch (Exception e) {
|
||||||
|
log.error("method=createTopic||param={}||operator={}||msg=add partition to db failed||errMsg=exception", dto, operator, e);
|
||||||
|
return Result.buildFromRSAndMsg(ResultStatus.MYSQL_OPERATE_FAILED, "Topic创建成功,但记录Partition到DB中失败,等待定时任务同步partition信息");
|
||||||
|
}
|
||||||
|
}
|
||||||
|
return createTopicRes;
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
|
|||||||
@@ -14,6 +14,8 @@ import java.util.Set;
|
|||||||
public interface PartitionService {
|
public interface PartitionService {
|
||||||
Result<Map<String, List<Partition>>> listPartitionsFromKafka(ClusterPhy clusterPhy);
|
Result<Map<String, List<Partition>>> listPartitionsFromKafka(ClusterPhy clusterPhy);
|
||||||
|
|
||||||
|
Result<List<Partition>> listPartitionsFromKafka(ClusterPhy clusterPhy, String topicName);
|
||||||
|
|
||||||
List<Partition> listPartitionByCluster(Long clusterPhyId);
|
List<Partition> listPartitionByCluster(Long clusterPhyId);
|
||||||
List<PartitionPO> listPartitionPOByCluster(Long clusterPhyId);
|
List<PartitionPO> listPartitionPOByCluster(Long clusterPhyId);
|
||||||
|
|
||||||
|
|||||||
@@ -98,6 +98,15 @@ public class PartitionServiceImpl extends BaseVersionControlService implements P
|
|||||||
return this.getPartitionsFromAdminClient(clusterPhy);
|
return this.getPartitionsFromAdminClient(clusterPhy);
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@Override
|
||||||
|
public Result<List<Partition>> listPartitionsFromKafka(ClusterPhy clusterPhy, String topicName) {
|
||||||
|
if (clusterPhy.getRunState().equals(ClusterRunStateEnum.RUN_ZK.getRunState())) {
|
||||||
|
return this.getPartitionsFromZKClientByClusterTopicName(clusterPhy,topicName);
|
||||||
|
}
|
||||||
|
return this.getPartitionsFromAdminClientByClusterTopicName(clusterPhy,topicName);
|
||||||
|
|
||||||
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
public List<Partition> listPartitionByCluster(Long clusterPhyId) {
|
public List<Partition> listPartitionByCluster(Long clusterPhyId) {
|
||||||
LambdaQueryWrapper<PartitionPO> lambdaQueryWrapper = new LambdaQueryWrapper<>();
|
LambdaQueryWrapper<PartitionPO> lambdaQueryWrapper = new LambdaQueryWrapper<>();
|
||||||
@@ -392,14 +401,12 @@ public class PartitionServiceImpl extends BaseVersionControlService implements P
|
|||||||
// 获取Topic列表
|
// 获取Topic列表
|
||||||
ListTopicsResult listTopicsResult = adminClient.listTopics(new ListTopicsOptions().timeoutMs(KafkaConstant.ADMIN_CLIENT_REQUEST_TIME_OUT_UNIT_MS).listInternal(true));
|
ListTopicsResult listTopicsResult = adminClient.listTopics(new ListTopicsOptions().timeoutMs(KafkaConstant.ADMIN_CLIENT_REQUEST_TIME_OUT_UNIT_MS).listInternal(true));
|
||||||
for (String topicName: listTopicsResult.names().get()) {
|
for (String topicName: listTopicsResult.names().get()) {
|
||||||
DescribeTopicsResult describeTopicsResult = adminClient.describeTopics(
|
Result<List<Partition>> partitionListRes = this.getPartitionsFromAdminClientByClusterTopicName(clusterPhy, topicName);
|
||||||
Arrays.asList(topicName),
|
if (partitionListRes.successful()){
|
||||||
new DescribeTopicsOptions().timeoutMs(KafkaConstant.ADMIN_CLIENT_REQUEST_TIME_OUT_UNIT_MS)
|
partitionMap.put(topicName, partitionListRes.getData());
|
||||||
);
|
}else {
|
||||||
|
return Result.buildFromIgnoreData(partitionListRes);
|
||||||
TopicDescription description = describeTopicsResult.all().get().get(topicName);
|
}
|
||||||
|
|
||||||
partitionMap.put(topicName, PartitionConverter.convert2PartitionList(clusterPhy.getId(), description));
|
|
||||||
}
|
}
|
||||||
|
|
||||||
return Result.buildSuc(partitionMap);
|
return Result.buildSuc(partitionMap);
|
||||||
@@ -416,13 +423,42 @@ public class PartitionServiceImpl extends BaseVersionControlService implements P
|
|||||||
try {
|
try {
|
||||||
List<String> topicNameList = kafkaZKDAO.getChildren(clusterPhy.getId(), TopicsZNode.path(), false);
|
List<String> topicNameList = kafkaZKDAO.getChildren(clusterPhy.getId(), TopicsZNode.path(), false);
|
||||||
for (String topicName: topicNameList) {
|
for (String topicName: topicNameList) {
|
||||||
PartitionMap zkPartitionMap = kafkaZKDAO.getData(clusterPhy.getId(), TopicZNode.path(topicName), PartitionMap.class);
|
Result<List<Partition>> partitionListRes = this.getPartitionsFromZKClientByClusterTopicName(clusterPhy, topicName);
|
||||||
|
if (partitionListRes.successful()){
|
||||||
|
partitionMap.put(topicName, partitionListRes.getData());
|
||||||
|
}
|
||||||
|
}
|
||||||
|
return Result.buildSuc(partitionMap);
|
||||||
|
} catch (Exception e) {
|
||||||
|
log.error("class=PartitionServiceImpl||method=getPartitionsFromZKClient||clusterPhyId={}||errMsg=exception", clusterPhy.getId(), e);
|
||||||
|
|
||||||
|
return Result.buildFromRSAndMsg(ResultStatus.KAFKA_OPERATE_FAILED, e.getMessage());
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
private Result<List<Partition>> getPartitionsFromAdminClientByClusterTopicName(ClusterPhy clusterPhy, String topicName) {
|
||||||
|
|
||||||
|
try {
|
||||||
|
AdminClient adminClient = kafkaAdminClient.getClient(clusterPhy.getId());
|
||||||
|
DescribeTopicsResult describeTopicsResult = adminClient.describeTopics(
|
||||||
|
Arrays.asList(topicName),
|
||||||
|
new DescribeTopicsOptions().timeoutMs(KafkaConstant.ADMIN_CLIENT_REQUEST_TIME_OUT_UNIT_MS)
|
||||||
|
);
|
||||||
|
TopicDescription description = describeTopicsResult.all().get().get(topicName);
|
||||||
|
return Result.buildSuc(PartitionConverter.convert2PartitionList(clusterPhy.getId(), description));
|
||||||
|
}catch (Exception e) {
|
||||||
|
log.error("class=PartitionServiceImpl||method=getPartitionsFromAdminClientByClusterTopicName||clusterPhyId={}||topicName={}||errMsg=exception", clusterPhy.getId(),topicName, e);
|
||||||
|
return Result.buildFailure(ResultStatus.KAFKA_OPERATE_FAILED);
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
private Result<List<Partition>> getPartitionsFromZKClientByClusterTopicName(ClusterPhy clusterPhy, String topicName) {
|
||||||
|
try {
|
||||||
|
PartitionMap zkPartitionMap = kafkaZKDAO.getData(clusterPhy.getId(), TopicZNode.path(topicName), PartitionMap.class);
|
||||||
List<Partition> partitionList = new ArrayList<>();
|
List<Partition> partitionList = new ArrayList<>();
|
||||||
List<String> partitionIdList = kafkaZKDAO.getChildren(clusterPhy.getId(), TopicPartitionsZNode.path(topicName), false);
|
List<String> partitionIdList = kafkaZKDAO.getChildren(clusterPhy.getId(), TopicPartitionsZNode.path(topicName), false);
|
||||||
for (String partitionId: partitionIdList) {
|
for (String partitionId: partitionIdList) {
|
||||||
PartitionState partitionState = kafkaZKDAO.getData(clusterPhy.getId(), TopicPartitionStateZNode.path(new TopicPartition(topicName, Integer.valueOf(partitionId))), PartitionState.class);
|
PartitionState partitionState = kafkaZKDAO.getData(clusterPhy.getId(), TopicPartitionStateZNode.path(new TopicPartition(topicName, Integer.valueOf(partitionId))), PartitionState.class);
|
||||||
|
|
||||||
Partition partition = new Partition();
|
Partition partition = new Partition();
|
||||||
partition.setClusterPhyId(clusterPhy.getId());
|
partition.setClusterPhyId(clusterPhy.getId());
|
||||||
partition.setTopicName(topicName);
|
partition.setTopicName(topicName);
|
||||||
@@ -430,17 +466,11 @@ public class PartitionServiceImpl extends BaseVersionControlService implements P
|
|||||||
partition.setLeaderBrokerId(partitionState.getLeader());
|
partition.setLeaderBrokerId(partitionState.getLeader());
|
||||||
partition.setInSyncReplicaList(partitionState.getIsr());
|
partition.setInSyncReplicaList(partitionState.getIsr());
|
||||||
partition.setAssignReplicaList(zkPartitionMap.getPartitionAssignReplicas(Integer.valueOf(partitionId)));
|
partition.setAssignReplicaList(zkPartitionMap.getPartitionAssignReplicas(Integer.valueOf(partitionId)));
|
||||||
|
|
||||||
partitionList.add(partition);
|
partitionList.add(partition);
|
||||||
}
|
}
|
||||||
|
return Result.buildSuc(partitionList);
|
||||||
partitionMap.put(topicName, partitionList);
|
|
||||||
}
|
|
||||||
|
|
||||||
return Result.buildSuc(partitionMap);
|
|
||||||
} catch (Exception e) {
|
} catch (Exception e) {
|
||||||
log.error("class=PartitionServiceImpl||method=getPartitionsFromZKClient||clusterPhyId={}||errMsg=exception", clusterPhy.getId(), e);
|
log.error("class=PartitionServiceImpl||method=getPartitionsFromZKClientByClusterTopicName||clusterPhyId={}||topicName={}||errMsg=exception", clusterPhy.getId(),topicName, e);
|
||||||
|
|
||||||
return Result.buildFromRSAndMsg(ResultStatus.KAFKA_OPERATE_FAILED, e.getMessage());
|
return Result.buildFromRSAndMsg(ResultStatus.KAFKA_OPERATE_FAILED, e.getMessage());
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|||||||
Reference in New Issue
Block a user