提交 97322eaf 编写于 作者: 王松松 提交者: Li Zhanhui

Fix Concurrent issue of StoreStatsService

上级 2047f94d
...@@ -21,6 +21,7 @@ import java.util.HashMap; ...@@ -21,6 +21,7 @@ import java.util.HashMap;
import java.util.LinkedList; import java.util.LinkedList;
import java.util.Map; import java.util.Map;
import java.util.concurrent.ConcurrentHashMap; import java.util.concurrent.ConcurrentHashMap;
import java.util.concurrent.ConcurrentMap;
import java.util.concurrent.atomic.AtomicLong; import java.util.concurrent.atomic.AtomicLong;
import java.util.concurrent.locks.ReentrantLock; import java.util.concurrent.locks.ReentrantLock;
import org.apache.rocketmq.common.ServiceThread; import org.apache.rocketmq.common.ServiceThread;
...@@ -42,9 +43,9 @@ public class StoreStatsService extends ServiceThread { ...@@ -42,9 +43,9 @@ public class StoreStatsService extends ServiceThread {
private final AtomicLong putMessageFailedTimes = new AtomicLong(0); private final AtomicLong putMessageFailedTimes = new AtomicLong(0);
private final Map<String, AtomicLong> putMessageTopicTimesTotal = private final ConcurrentMap<String, AtomicLong> putMessageTopicTimesTotal =
new ConcurrentHashMap<String, AtomicLong>(128); new ConcurrentHashMap<String, AtomicLong>(128);
private final Map<String, AtomicLong> putMessageTopicSizeTotal = private final ConcurrentMap<String, AtomicLong> putMessageTopicSizeTotal =
new ConcurrentHashMap<String, AtomicLong>(128); new ConcurrentHashMap<String, AtomicLong>(128);
private final AtomicLong getMessageTimesTotalFound = new AtomicLong(0); private final AtomicLong getMessageTimesTotalFound = new AtomicLong(0);
...@@ -545,7 +546,10 @@ public class StoreStatsService extends ServiceThread { ...@@ -545,7 +546,10 @@ public class StoreStatsService extends ServiceThread {
AtomicLong rs = putMessageTopicSizeTotal.get(topic); AtomicLong rs = putMessageTopicSizeTotal.get(topic);
if (null == rs) { if (null == rs) {
rs = new AtomicLong(0); rs = new AtomicLong(0);
putMessageTopicSizeTotal.put(topic, rs); AtomicLong previous = putMessageTopicSizeTotal.putIfAbsent(topic, rs);
if (previous != null) {
rs = previous;
}
} }
return rs; return rs;
} }
...@@ -554,7 +558,10 @@ public class StoreStatsService extends ServiceThread { ...@@ -554,7 +558,10 @@ public class StoreStatsService extends ServiceThread {
AtomicLong rs = putMessageTopicTimesTotal.get(topic); AtomicLong rs = putMessageTopicTimesTotal.get(topic);
if (null == rs) { if (null == rs) {
rs = new AtomicLong(0); rs = new AtomicLong(0);
putMessageTopicTimesTotal.put(topic, rs); AtomicLong previous = putMessageTopicTimesTotal.putIfAbsent(topic, rs);
if (previous != null) {
rs = previous;
}
} }
return rs; return rs;
} }
......
/*
* 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.store;
import java.util.concurrent.BrokenBarrierException;
import java.util.concurrent.CountDownLatch;
import java.util.concurrent.CyclicBarrier;
import java.util.concurrent.atomic.AtomicLong;
import java.util.concurrent.atomic.AtomicReference;
import org.junit.Test;
public class StoreStatsServiceTest {
@Test
public void getSinglePutMessageTopicSizeTotal() throws Exception {
final StoreStatsService storeStatsService = new StoreStatsService();
int num = Runtime.getRuntime().availableProcessors() * 2;
for (int j = 0; j < 100; j++) {
final AtomicReference<AtomicLong> reference = new AtomicReference<>(null);
final CountDownLatch latch = new CountDownLatch(num);
final CyclicBarrier barrier = new CyclicBarrier(num);
for (int i = 0; i < num; i++) {
new Thread(new Runnable() {
@Override
public void run() {
try {
barrier.await();
AtomicLong atomicLong = storeStatsService.getSinglePutMessageTopicSizeTotal("test");
if (reference.compareAndSet(null, atomicLong)) {
} else if (reference.get() != atomicLong) {
throw new RuntimeException("Reference should be same!");
}
} catch (InterruptedException | BrokenBarrierException e) {
e.printStackTrace();
} finally {
latch.countDown();
}
}
}).start();
}
latch.await();
}
}
@Test
public void getSinglePutMessageTopicTimesTotal() throws Exception {
final StoreStatsService storeStatsService = new StoreStatsService();
int num = Runtime.getRuntime().availableProcessors() * 2;
for (int j = 0; j < 100; j++) {
final AtomicReference<AtomicLong> reference = new AtomicReference<>(null);
final CountDownLatch latch = new CountDownLatch(num);
final CyclicBarrier barrier = new CyclicBarrier(num);
for (int i = 0; i < num; i++) {
new Thread(new Runnable() {
@Override
public void run() {
try {
barrier.await();
AtomicLong atomicLong = storeStatsService.getSinglePutMessageTopicTimesTotal("test");
if (reference.compareAndSet(null, atomicLong)) {
} else if (reference.get() != atomicLong) {
throw new RuntimeException("Reference should be same!");
}
} catch (InterruptedException | BrokenBarrierException e) {
e.printStackTrace();
} finally {
latch.countDown();
}
}
}).start();
}
latch.await();
}
}
}
\ No newline at end of file
Markdown is supported
0% .
You are about to add 0 people to the discussion. Proceed with caution.
先完成此消息的编辑!
想要评论请 注册