forked from apache/pulsar
-
Notifications
You must be signed in to change notification settings - Fork 0
Commit
This commit does not belong to any branch on this repository, and may belong to a fork outside of the repository.
[Pulsar SQL] Add max split entry queue size bytes limitation (apache#…
…9628) ### Motivation In Pulsar SQL, there are two configurations `pulsar.max-split-entry-queue-size` and `pulsar.max-split-message-queue-size` to control the entry queue and message queue capacity, but some entries are so big some are small, it's hard to control the queue size bytes and the message queue size bytes. ### Modifications Add a new configuration `pulsar.max-split-queue-cache-size` to control the entry queue size bytes and the message queue size bytes. Half of this configuration will assign to entry queue size bytes and the left quota assign to message queue size bytes.
- Loading branch information
Showing
14 changed files
with
525 additions
and
13 deletions.
There are no files selected for viewing
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
47 changes: 47 additions & 0 deletions
47
...sql/presto-pulsar/src/main/java/org/apache/pulsar/sql/presto/util/CacheSizeAllocator.java
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Original file line number | Diff line number | Diff line change |
---|---|---|
@@ -0,0 +1,47 @@ | ||
/** | ||
* 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.pulsar.sql.presto.util; | ||
|
||
/** | ||
* Cache size allocator. | ||
*/ | ||
public interface CacheSizeAllocator { | ||
|
||
/** | ||
* Get available cache size. | ||
* | ||
* @return available cache size | ||
*/ | ||
public long getAvailableCacheSize(); | ||
|
||
/** | ||
* Cost available cache. | ||
* | ||
* @param size allocate size | ||
*/ | ||
public void allocate(long size); | ||
|
||
/** | ||
* Release allocated cache size. | ||
* | ||
* @param size release size | ||
*/ | ||
public void release(long size); | ||
|
||
} |
84 changes: 84 additions & 0 deletions
84
...to-pulsar/src/main/java/org/apache/pulsar/sql/presto/util/NoStrictCacheSizeAllocator.java
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Original file line number | Diff line number | Diff line change |
---|---|---|
@@ -0,0 +1,84 @@ | ||
/** | ||
* 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.pulsar.sql.presto.util; | ||
|
||
import java.util.concurrent.atomic.LongAdder; | ||
import java.util.concurrent.locks.ReentrantLock; | ||
import java.util.concurrent.locks.ReentrantReadWriteLock; | ||
|
||
/** | ||
* Cache size allocator. | ||
*/ | ||
public class NoStrictCacheSizeAllocator implements CacheSizeAllocator { | ||
|
||
private final long maxCacheSize; | ||
private final LongAdder availableCacheSize; | ||
private final ReentrantLock lock; | ||
|
||
public NoStrictCacheSizeAllocator(long maxCacheSize) { | ||
this.maxCacheSize = maxCacheSize; | ||
this.availableCacheSize = new LongAdder(); | ||
this.availableCacheSize.add(maxCacheSize); | ||
this.lock = new ReentrantLock(); | ||
} | ||
|
||
public long getAvailableCacheSize() { | ||
if (availableCacheSize.longValue() < 0) { | ||
return 0; | ||
} | ||
return availableCacheSize.longValue(); | ||
} | ||
|
||
/** | ||
* This operation will cost available cache size. | ||
* if the request size exceed the available size, it's should be allowed, | ||
* because maybe one entry size exceed the size and | ||
* the query must be finished, the available size will become invalid. | ||
* | ||
* @param size allocate size | ||
*/ | ||
public void allocate(long size) { | ||
try { | ||
lock.lock(); | ||
availableCacheSize.add(-size); | ||
} finally { | ||
lock.unlock(); | ||
} | ||
} | ||
|
||
/** | ||
* This method used to release used cache size and add available cache size. | ||
* in normal case, the available size shouldn't exceed max cache size. | ||
* | ||
* @param size release size | ||
*/ | ||
public void release(long size) { | ||
try { | ||
lock.lock(); | ||
availableCacheSize.add(size); | ||
if (availableCacheSize.longValue() > maxCacheSize) { | ||
availableCacheSize.reset(); | ||
availableCacheSize.add(maxCacheSize); | ||
} | ||
} finally { | ||
lock.unlock(); | ||
} | ||
} | ||
|
||
} |
Oops, something went wrong.