[GitHub] ivankelly commented on a change in pull request #1236: Issue #570: make changes to SyncThread/checkpoint logic.
ivankelly commented on a change in pull request #1236: Issue #570: make changes to SyncThread/checkpoint logic. URL: https://github.com/apache/bookkeeper/pull/1236#discussion_r174082113 ## File path: bookkeeper-server/src/main/java/org/apache/bookkeeper/bookie/SyncThread.java ## @@ -108,14 +103,13 @@ public void startCheckpoint(Checkpoint checkpoint) { }); } -public Future requestFlush() { +public Future requestFlush() { Review comment: nit: why did you remove the generic here? This is an automated message from the Apache Git Service. To respond to the message, please log on GitHub and use the URL above to go to the specific comment. For queries about this service, please contact Infrastructure at: us...@infra.apache.org With regards, Apache Git Services
[GitHub] ivankelly commented on a change in pull request #1236: Issue #570: make changes to SyncThread/checkpoint logic.
ivankelly commented on a change in pull request #1236: Issue #570: make changes to SyncThread/checkpoint logic. URL: https://github.com/apache/bookkeeper/pull/1236#discussion_r173739266 ## File path: bookkeeper-server/src/main/java/org/apache/bookkeeper/conf/ServerConfiguration.java ## @@ -179,6 +179,11 @@ // Stats protected static final String ENABLE_TASK_EXECUTION_STATS = "enableTaskExecutionStats"; +/* + * config specifying if the entrylog per ledger is enabled or not. + */ +protected static final String ENTRY_LOG_PERLEDGER_ENABLED = "entryLogPerLedgerEnabled"; Review comment: the setting name is entryLogPerLedgerEnabled, not entryLogPerledgerEnabled. It's the variable I want updated to be consistent with the setting name, as the setting name treats Per and Ledger as separate words. As they are. This is an automated message from the Apache Git Service. To respond to the message, please log on GitHub and use the URL above to go to the specific comment. For queries about this service, please contact Infrastructure at: us...@infra.apache.org With regards, Apache Git Services
[GitHub] ivankelly commented on a change in pull request #1236: Issue #570: make changes to SyncThread/checkpoint logic.
ivankelly commented on a change in pull request #1236: Issue #570: make changes to SyncThread/checkpoint logic. URL: https://github.com/apache/bookkeeper/pull/1236#discussion_r173431473 ## File path: bookkeeper-server/src/main/java/org/apache/bookkeeper/meta/LedgerManagerFactory.java ## @@ -94,7 +94,7 @@ LedgerUnderreplicationManager newLedgerUnderreplicationManager() * @param lm *Layout manager */ Review comment: This is already in. This is an automated message from the Apache Git Service. To respond to the message, please log on GitHub and use the URL above to go to the specific comment. For queries about this service, please contact Infrastructure at: us...@infra.apache.org With regards, Apache Git Services
[GitHub] ivankelly commented on a change in pull request #1236: Issue #570: make changes to SyncThread/checkpoint logic.
ivankelly commented on a change in pull request #1236: Issue #570: make changes to SyncThread/checkpoint logic. URL: https://github.com/apache/bookkeeper/pull/1236#discussion_r173429077 ## File path: bookkeeper-server/src/main/java/org/apache/bookkeeper/bookie/InterleavedLedgerStorage.java ## @@ -331,9 +333,16 @@ private void flushOrCheckpoint(boolean isCheckpointFlush) } try { -// if it is just a checkpoint flush, we just flush rotated entry log files -// in entry logger. -if (isCheckpointFlush) { +/* + * if it is just a checkpoint flush and if entryLogPerLedger is not + * enabled, then we just flush rotated entry log files in entry + * logger. + * + * In the case of entryLogPerLedgerEnabled we need to flush both + * rotatedlogs and currentlogs. Hence we call entryLogger.flush in + * the case of entrylogperledgerenabled. + */ +if (isCheckpointFlush && !entryLogPerLedgerEnabled) { Review comment: So effectively, entryLogPerLedgerEnabled disables checkpoint() on EntryLogger. Don't add another boolean flag in this method. There's already 2 in play, and this method needs to be refactored badly. Instead, handle it in the constructor. ``` if (conf.isEntryLogPerLedgerEnabled()) { entryLogger = new EntryLogger(conf, ledgerDirsManager, this) { @Override void checkpoint() throws IOException { // Add comment explaining why you always need to flush for per ledger flush(); } }; } else { entryLogger = new EntryLogger(conf, ledgerDirsManager, this); } ``` Eventually when the whole feature in, this can be refactored into it's own entrylogger. This is an automated message from the Apache Git Service. To respond to the message, please log on GitHub and use the URL above to go to the specific comment. For queries about this service, please contact Infrastructure at: us...@infra.apache.org With regards, Apache Git Services
[GitHub] ivankelly commented on a change in pull request #1236: Issue #570: make changes to SyncThread/checkpoint logic.
ivankelly commented on a change in pull request #1236: Issue #570: make changes to SyncThread/checkpoint logic. URL: https://github.com/apache/bookkeeper/pull/1236#discussion_r173431258 ## File path: bookkeeper-server/src/main/java/org/apache/bookkeeper/conf/ServerConfiguration.java ## @@ -179,6 +179,11 @@ // Stats protected static final String ENABLE_TASK_EXECUTION_STATS = "enableTaskExecutionStats"; +/* + * config specifying if the entrylog per ledger is enabled or not. + */ +protected static final String ENTRY_LOG_PERLEDGER_ENABLED = "entryLogPerLedgerEnabled"; Review comment: Add a _ between PER and LEDGER This is an automated message from the Apache Git Service. To respond to the message, please log on GitHub and use the URL above to go to the specific comment. For queries about this service, please contact Infrastructure at: us...@infra.apache.org With regards, Apache Git Services
[GitHub] ivankelly commented on a change in pull request #1236: Issue #570: make changes to SyncThread/checkpoint logic.
ivankelly commented on a change in pull request #1236: Issue #570: make changes to SyncThread/checkpoint logic. URL: https://github.com/apache/bookkeeper/pull/1236#discussion_r173434239 ## File path: bookkeeper-server/src/test/java/org/apache/bookkeeper/bookie/LedgerStorageCheckpointTest.java ## @@ -0,0 +1,580 @@ +/** + * + * 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.bookkeeper.bookie; + +import java.io.File; +import java.io.FileInputStream; +import java.io.IOException; +import java.nio.ByteBuffer; +import java.util.Enumeration; +import java.util.List; +import java.util.Random; +import java.util.concurrent.CountDownLatch; +import java.util.concurrent.ExecutorService; +import java.util.concurrent.Executors; +import java.util.concurrent.TimeUnit; +import java.util.concurrent.atomic.AtomicBoolean; + +import org.apache.bookkeeper.bookie.Journal.LastLogMark; +import org.apache.bookkeeper.client.BookKeeper; +import org.apache.bookkeeper.client.BookKeeper.DigestType; +import org.apache.bookkeeper.client.LedgerEntry; +import org.apache.bookkeeper.client.LedgerHandle; +import org.apache.bookkeeper.conf.ClientConfiguration; +import org.apache.bookkeeper.conf.ServerConfiguration; +import org.apache.bookkeeper.conf.TestBKConfiguration; +import org.apache.bookkeeper.proto.BookieServer; +import org.apache.bookkeeper.test.BookKeeperClusterTestCase; +import org.apache.bookkeeper.test.PortManager; +import org.junit.Assert; +import org.junit.Rule; +import org.junit.Test; +import org.junit.rules.TestName; +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; + +/** + * LedgerStorageCheckpointTest. + * + */ +public class LedgerStorageCheckpointTest extends BookKeeperClusterTestCase { +private static final Logger LOG = LoggerFactory +.getLogger(LedgerStorageCheckpointTest.class); + +@Rule +public final TestName runtime = new TestName(); + +public LedgerStorageCheckpointTest() { +super(0); +} + +private LogMark readLastMarkFile(File lastMarkFile) throws IOException { +byte buff[] = new byte[16]; +ByteBuffer bb = ByteBuffer.wrap(buff); +LogMark rolledLogMark = new LogMark(); +FileInputStream fis = new FileInputStream(lastMarkFile); +int bytesRead = fis.read(buff); +fis.close(); +if (bytesRead != 16) { +throw new IOException("Couldn't read enough bytes from lastMark." + " Wanted " + 16 + ", got " + bytesRead); +} +bb.clear(); +rolledLogMark.readLogMark(bb); +return rolledLogMark; +} + +/* + * In this testcase, InterleavedLedgerStorage is used and validate if the + * checkpoint is called for every flushinterval period. + */ +@Test +public void testPeriodicCheckpointForInterleavedLedgerStorage() throws Exception { + testPeriodicCheckpointForLedgerStorage(InterleavedLedgerStorage.class.getName()); +} + +/* + * In this testcase, SortedLedgerStorage is used and validate if the + * checkpoint is called for every flushinterval period. + */ +@Test +public void testPeriodicCheckpointForSortedLedgerStorage() throws Exception { + testPeriodicCheckpointForLedgerStorage(SortedLedgerStorage.class.getName()); +} + +public void testPeriodicCheckpointForLedgerStorage(String ledgerStorageClassName) throws Exception { Review comment: What are you actually testing here? There are two changes in the main code. They should be tested separately. For the SyncThread change, you just need to validate that if perLedgerEntryLog is true, it runs periodically. You only need to construct a SyncThread for this. LedgerStorage and everything else can be mocked. For the InterleavedLedgerStorage, you want to check that if perLedgerEntryLog is set, then flush always gets called instead of checkpoint. Similarly this can be done with mocks. This is an automated message from the Apache Git Service. To respond to the message, please log on GitHub and use the URL above to go to the specific comment. For queries about this service, please contact
[GitHub] ivankelly commented on a change in pull request #1236: Issue #570: make changes to SyncThread/checkpoint logic.
ivankelly commented on a change in pull request #1236: Issue #570: make changes to SyncThread/checkpoint logic. URL: https://github.com/apache/bookkeeper/pull/1236#discussion_r173433251 ## File path: bookkeeper-server/src/test/java/org/apache/bookkeeper/bookie/LedgerStorageCheckpointTest.java ## @@ -0,0 +1,580 @@ +/** + * + * 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.bookkeeper.bookie; + +import java.io.File; +import java.io.FileInputStream; +import java.io.IOException; +import java.nio.ByteBuffer; +import java.util.Enumeration; +import java.util.List; +import java.util.Random; +import java.util.concurrent.CountDownLatch; +import java.util.concurrent.ExecutorService; +import java.util.concurrent.Executors; +import java.util.concurrent.TimeUnit; +import java.util.concurrent.atomic.AtomicBoolean; + +import org.apache.bookkeeper.bookie.Journal.LastLogMark; +import org.apache.bookkeeper.client.BookKeeper; +import org.apache.bookkeeper.client.BookKeeper.DigestType; +import org.apache.bookkeeper.client.LedgerEntry; +import org.apache.bookkeeper.client.LedgerHandle; +import org.apache.bookkeeper.conf.ClientConfiguration; +import org.apache.bookkeeper.conf.ServerConfiguration; +import org.apache.bookkeeper.conf.TestBKConfiguration; +import org.apache.bookkeeper.proto.BookieServer; +import org.apache.bookkeeper.test.BookKeeperClusterTestCase; +import org.apache.bookkeeper.test.PortManager; +import org.junit.Assert; +import org.junit.Rule; +import org.junit.Test; +import org.junit.rules.TestName; +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; + +/** + * LedgerStorageCheckpointTest. + * + */ +public class LedgerStorageCheckpointTest extends BookKeeperClusterTestCase { +private static final Logger LOG = LoggerFactory +.getLogger(LedgerStorageCheckpointTest.class); + +@Rule +public final TestName runtime = new TestName(); + +public LedgerStorageCheckpointTest() { +super(0); +} + +private LogMark readLastMarkFile(File lastMarkFile) throws IOException { +byte buff[] = new byte[16]; +ByteBuffer bb = ByteBuffer.wrap(buff); +LogMark rolledLogMark = new LogMark(); +FileInputStream fis = new FileInputStream(lastMarkFile); +int bytesRead = fis.read(buff); +fis.close(); +if (bytesRead != 16) { +throw new IOException("Couldn't read enough bytes from lastMark." + " Wanted " + 16 + ", got " + bytesRead); +} +bb.clear(); +rolledLogMark.readLogMark(bb); +return rolledLogMark; +} + +/* + * In this testcase, InterleavedLedgerStorage is used and validate if the + * checkpoint is called for every flushinterval period. + */ +@Test +public void testPeriodicCheckpointForInterleavedLedgerStorage() throws Exception { + testPeriodicCheckpointForLedgerStorage(InterleavedLedgerStorage.class.getName()); +} + +/* + * In this testcase, SortedLedgerStorage is used and validate if the + * checkpoint is called for every flushinterval period. + */ +@Test +public void testPeriodicCheckpointForSortedLedgerStorage() throws Exception { + testPeriodicCheckpointForLedgerStorage(SortedLedgerStorage.class.getName()); +} + +public void testPeriodicCheckpointForLedgerStorage(String ledgerStorageClassName) throws Exception { +File tmpDir = createTempDir("DiskCheck", "test"); + +final ServerConfiguration conf = TestBKConfiguration.newServerConfiguration() +.setZkServers(zkUtil.getZooKeeperConnectString()) +.setZkTimeout(5000) +.setJournalDirName(tmpDir.getPath()) +.setLedgerDirNames(new String[] { tmpDir.getPath() }) +.setAutoRecoveryDaemonEnabled(false) +.setFlushInterval(2000) +.setBookiePort(PortManager.nextFreePort()) +// entrylog per ledger is enabled +.setEntryLogPerLedgerEnabled(true) +.setLedgerStorageClass(ledgerStorageClassName); +Assert.assertEquals("Number of JournalDirs", 1, conf.getJournalDirs().length); +// we k
[GitHub] ivankelly commented on a change in pull request #1236: Issue #570: make changes to SyncThread/checkpoint logic.
ivankelly commented on a change in pull request #1236: Issue #570: make changes to SyncThread/checkpoint logic. URL: https://github.com/apache/bookkeeper/pull/1236#discussion_r173430929 ## File path: bookkeeper-server/src/main/java/org/apache/bookkeeper/bookie/Bookie.java ## @@ -696,14 +698,29 @@ public Bookie(ServerConfiguration conf, StatsLogger statsLogger) ledgerStorage = LedgerStorageFactory.createLedgerStorage(ledgerStorageClass); syncThread = new SyncThread(conf, getLedgerDirsListener(), ledgerStorage, checkpointSource); +Checkpointer checkpointer; +/* + * with this change https://github.com/apache/bookkeeper/pull/677, + * LedgerStorage drives the checkpoint logic. But with multiple entry + * logs, checkpoint logic based on a entry log is not possible, hence it + * needs to be timebased recurring thing and it is driven by SyncThread. + * SyncThread.start does that and it is started in Bookie.start method. + */ +if (entryLogPerLedgerEnabled) { Review comment: What you're basically doing here is creating two implementations of syncthread. One which is periodic, and one which uses checkpoint. This should be handled at the class level, rather than with boolean flags. If you don't want to go to the trouble of creating a bunch of new classes, then do. ``` if (conf.isEntryLogPerLedgerEnabled()) { syncThread = new SyncThread(conf, getLedgerDirsListener(), ledgerStorage, checkpointSource) { @Override public void checkpoint(Checkpoint checkpoint) { // A comment for why it's disabled } @Override public void start() { syncThread.scheduleCheckpointAtFixedRate(conf.getFlushInterval()); } }; } else { syncThread = new SyncThread(conf, getLedgerDirsListener(), ledgerStorage, checkpointSource); } ``` Add a start method to SyncThread and always call it in Bookie#start. There used to be one, and it makes sense for there to be one, since it is called SyncThread. This is an automated message from the Apache Git Service. To respond to the message, please log on GitHub and use the URL above to go to the specific comment. For queries about this service, please contact Infrastructure at: us...@infra.apache.org With regards, Apache Git Services