Block on BufferReleaseChannel when out of buffers
Bug: 294133380
Flag: com.android.graphics.libgui.flags.buffer_release_channel
Test: BLASTBufferQueueTest
Change-Id: I4ffec81ffb9c26546cc50176f3c44ffe6eb90b75
diff --git a/libs/gui/BLASTBufferQueue.cpp b/libs/gui/BLASTBufferQueue.cpp
index 25e6a52..49f4cba 100644
--- a/libs/gui/BLASTBufferQueue.cpp
+++ b/libs/gui/BLASTBufferQueue.cpp
@@ -50,9 +50,28 @@
using namespace std::chrono_literals;
namespace {
+
+#if COM_ANDROID_GRAPHICS_LIBGUI_FLAGS(BUFFER_RELEASE_CHANNEL)
+// RAII wrapper to defer arbitrary work until the Deferred instance is deleted.
+template <class F>
+class Deferred {
+public:
+ explicit Deferred(F f) : mF{std::move(f)} {}
+
+ ~Deferred() { mF(); }
+
+ Deferred(const Deferred&) = delete;
+ Deferred& operator=(const Deferred&) = delete;
+
+private:
+ F mF;
+};
+#endif
+
inline const char* boolToString(bool b) {
return b ? "true" : "false";
}
+
} // namespace
namespace android {
@@ -77,12 +96,6 @@
std::unique_lock _lock{mutex}; \
base::ScopedLockAssertion assumeLocked(mutex);
-#if COM_ANDROID_GRAPHICS_LIBGUI_FLAGS(BUFFER_RELEASE_CHANNEL)
-static ReleaseBufferCallback EMPTY_RELEASE_CALLBACK =
- [](const ReleaseCallbackId&, const sp<Fence>& /*releaseFence*/,
- std::optional<uint32_t> /*currentMaxAcquiredBufferCount*/) {};
-#endif
-
void BLASTBufferItemConsumer::onDisconnect() {
Mutex::Autolock lock(mMutex);
mPreviouslyConnected = mCurrentlyConnected;
@@ -225,9 +238,8 @@
this);
#if COM_ANDROID_GRAPHICS_LIBGUI_FLAGS(BUFFER_RELEASE_CHANNEL)
- std::unique_ptr<gui::BufferReleaseChannel::ConsumerEndpoint> bufferReleaseConsumer;
- gui::BufferReleaseChannel::open(mName, bufferReleaseConsumer, mBufferReleaseProducer);
- mBufferReleaseReader = std::make_shared<BufferReleaseReader>(std::move(bufferReleaseConsumer));
+ gui::BufferReleaseChannel::open(mName, mBufferReleaseConsumer, mBufferReleaseProducer);
+ mBufferReleaseReader.emplace(*this);
#endif
BQA_LOGV("BLASTBufferQueue created");
@@ -260,7 +272,7 @@
// safe default, most producers are expected to override this
mProducer->setMaxDequeuedBufferCount(2);
#if COM_ANDROID_GRAPHICS_LIBGUI_FLAGS(BUFFER_RELEASE_CHANNEL)
- mBufferReleaseThread.start(sp<BLASTBufferQueue>::fromExisting(this));
+ mBufferReleaseThread.emplace(sp<BLASTBufferQueue>::fromExisting(this));
#endif
}
@@ -636,7 +648,7 @@
#if COM_ANDROID_GRAPHICS_LIBGUI_FLAGS(BUFFER_RELEASE_CHANNEL)
ReleaseBufferCallback releaseBufferCallback =
- applyTransaction ? EMPTY_RELEASE_CALLBACK : makeReleaseBufferCallbackThunk();
+ applyTransaction ? nullptr : makeReleaseBufferCallbackThunk();
#else
auto releaseBufferCallback = makeReleaseBufferCallbackThunk();
#endif
@@ -1137,6 +1149,24 @@
#endif
};
+#if COM_ANDROID_GRAPHICS_LIBGUI_FLAGS(BUFFER_RELEASE_CHANNEL)
+class BBQBufferQueueCore : public BufferQueueCore {
+public:
+ explicit BBQBufferQueueCore(const wp<BLASTBufferQueue>& bbq) : mBLASTBufferQueue{bbq} {}
+
+ void notifyBufferReleased() const override {
+ sp<BLASTBufferQueue> bbq = mBLASTBufferQueue.promote();
+ if (!bbq) {
+ return;
+ }
+ bbq->mBufferReleaseReader->interruptBlockingRead();
+ }
+
+private:
+ wp<BLASTBufferQueue> mBLASTBufferQueue;
+};
+#endif
+
// Extends the BufferQueueProducer to create a wrapper around the listener so the listener calls
// can be non-blocking when the producer is in the client process.
class BBQBufferQueueProducer : public BufferQueueProducer {
@@ -1188,6 +1218,44 @@
return BufferQueueProducer::query(what, value);
}
+#if COM_ANDROID_GRAPHICS_LIBGUI_FLAGS(BUFFER_RELEASE_CHANNEL)
+ status_t waitForBufferRelease(std::unique_lock<std::mutex>& bufferQueueLock,
+ nsecs_t timeout) const override {
+ sp<BLASTBufferQueue> bbq = mBLASTBufferQueue.promote();
+ if (!bbq) {
+ return OK;
+ }
+
+ // BufferQueue has already checked if we have a free buffer. If there's an unread interrupt,
+ // we want to ignore it. This must be done before unlocking the BufferQueue lock to ensure
+ // we don't miss an interrupt.
+ bbq->mBufferReleaseReader->clearInterrupts();
+ bbq->mThreadsBlockingOnDequeue++;
+ bufferQueueLock.unlock();
+ Deferred cleanup{[&]() {
+ bufferQueueLock.lock();
+ bbq->mThreadsBlockingOnDequeue--;
+ }};
+
+ ATRACE_FORMAT("waiting for free buffer");
+ ReleaseCallbackId id;
+ sp<Fence> fence;
+ uint32_t maxAcquiredBufferCount;
+ status_t status =
+ bbq->mBufferReleaseReader->readBlocking(id, fence, maxAcquiredBufferCount, timeout);
+ if (status == TIMED_OUT) {
+ return TIMED_OUT;
+ } else if (status != OK) {
+ // Waiting was interrupted or an error occurred. BufferQueueProducer will check if we
+ // have a free buffer and call this method again if not.
+ return OK;
+ }
+
+ bbq->releaseBufferCallback(id, fence, maxAcquiredBufferCount);
+ return OK;
+ }
+#endif
+
private:
const wp<BLASTBufferQueue> mBLASTBufferQueue;
};
@@ -1201,14 +1269,18 @@
LOG_ALWAYS_FATAL_IF(outProducer == nullptr, "BLASTBufferQueue: outProducer must not be NULL");
LOG_ALWAYS_FATAL_IF(outConsumer == nullptr, "BLASTBufferQueue: outConsumer must not be NULL");
- sp<BufferQueueCore> core(new BufferQueueCore());
+#if COM_ANDROID_GRAPHICS_LIBGUI_FLAGS(BUFFER_RELEASE_CHANNEL)
+ auto core = sp<BBQBufferQueueCore>::make(this);
+#else
+ auto core = sp<BufferQueueCore>::make();
+#endif
LOG_ALWAYS_FATAL_IF(core == nullptr, "BLASTBufferQueue: failed to create BufferQueueCore");
- sp<IGraphicBufferProducer> producer(new BBQBufferQueueProducer(core, this));
+ auto producer = sp<BBQBufferQueueProducer>::make(core, this);
LOG_ALWAYS_FATAL_IF(producer == nullptr,
"BLASTBufferQueue: failed to create BBQBufferQueueProducer");
- sp<BufferQueueConsumer> consumer(new BufferQueueConsumer(core));
+ auto consumer = sp<BufferQueueConsumer>::make(core);
consumer->setAllowExtraAcquire(true);
LOG_ALWAYS_FATAL_IF(consumer == nullptr,
"BLASTBufferQueue: failed to create BufferQueueConsumer");
@@ -1273,10 +1345,8 @@
#if COM_ANDROID_GRAPHICS_LIBGUI_FLAGS(BUFFER_RELEASE_CHANNEL)
-BLASTBufferQueue::BufferReleaseReader::BufferReleaseReader(
- std::unique_ptr<gui::BufferReleaseChannel::ConsumerEndpoint> endpoint)
- : mEndpoint{std::move(endpoint)} {
- mEpollFd = android::base::unique_fd{epoll_create1(0)};
+BLASTBufferQueue::BufferReleaseReader::BufferReleaseReader(BLASTBufferQueue& bbq) : mBbq{bbq} {
+ mEpollFd = android::base::unique_fd{epoll_create1(EPOLL_CLOEXEC)};
LOG_ALWAYS_FATAL_IF(!mEpollFd.ok(),
"Failed to create buffer release epoll file descriptor. errno=%d "
"message='%s'",
@@ -1284,9 +1354,9 @@
epoll_event registerEndpointFd{};
registerEndpointFd.events = EPOLLIN;
- registerEndpointFd.data.fd = mEndpoint->getFd();
- status_t status =
- epoll_ctl(mEpollFd.get(), EPOLL_CTL_ADD, mEndpoint->getFd(), ®isterEndpointFd);
+ registerEndpointFd.data.fd = mBbq.mBufferReleaseConsumer->getFd();
+ status_t status = epoll_ctl(mEpollFd.get(), EPOLL_CTL_ADD, mBbq.mBufferReleaseConsumer->getFd(),
+ ®isterEndpointFd);
LOG_ALWAYS_FATAL_IF(status == -1,
"Failed to register buffer release consumer file descriptor with epoll. "
"errno=%d message='%s'",
@@ -1308,78 +1378,153 @@
errno, strerror(errno));
}
-BLASTBufferQueue::BufferReleaseReader& BLASTBufferQueue::BufferReleaseReader::operator=(
- BufferReleaseReader&& other) {
- if (this != &other) {
- ftl::FakeGuard guard{mMutex};
- ftl::FakeGuard otherGuard{other.mMutex};
- mEndpoint = std::move(other.mEndpoint);
- mEpollFd = std::move(other.mEpollFd);
- mEventFd = std::move(other.mEventFd);
- }
- return *this;
-}
-
status_t BLASTBufferQueue::BufferReleaseReader::readBlocking(ReleaseCallbackId& outId,
sp<Fence>& outFence,
- uint32_t& outMaxAcquiredBufferCount) {
+ uint32_t& outMaxAcquiredBufferCount,
+ nsecs_t timeout) {
+ // TODO(b/363290953) epoll_wait only has millisecond timeout precision. If timeout is less than
+ // 1ms, then we round timeout up to 1ms. Otherwise, we round timeout to the nearest
+ // millisecond. Once epoll_pwait2 can be used in libgui, we can specify timeout with nanosecond
+ // precision.
+ int timeoutMs = -1;
+ if (timeout == 0) {
+ timeoutMs = 0;
+ } else if (timeout > 0) {
+ const int nsPerMs = 1000000;
+ if (timeout < nsPerMs) {
+ timeoutMs = 1;
+ } else {
+ timeoutMs = static_cast<int>(
+ std::chrono::round<std::chrono::milliseconds>(std::chrono::nanoseconds{timeout})
+ .count());
+ }
+ }
+
epoll_event event{};
- while (true) {
- int eventCount = epoll_wait(mEpollFd.get(), &event, 1 /* maxevents */, -1 /* timeout */);
- if (eventCount == 1) {
- break;
- }
- if (eventCount == -1 && errno != EINTR) {
- ALOGE("epoll_wait error while waiting for buffer release. errno=%d message='%s'", errno,
- strerror(errno));
- }
+ int eventCount;
+ do {
+ eventCount = epoll_wait(mEpollFd.get(), &event, 1 /*maxevents*/, timeoutMs);
+ } while (eventCount == -1 && errno != EINTR);
+
+ if (eventCount == -1) {
+ ALOGE("epoll_wait error while waiting for buffer release. errno=%d message='%s'", errno,
+ strerror(errno));
+ return UNKNOWN_ERROR;
+ }
+
+ if (eventCount == 0) {
+ return TIMED_OUT;
}
if (event.data.fd == mEventFd.get()) {
- uint64_t value;
- if (read(mEventFd.get(), &value, sizeof(uint64_t)) == -1 && errno != EWOULDBLOCK) {
- ALOGE("error while reading from eventfd. errno=%d message='%s'", errno,
- strerror(errno));
- }
+ clearInterrupts();
return WOULD_BLOCK;
}
- std::lock_guard lock{mMutex};
- return mEndpoint->readReleaseFence(outId, outFence, outMaxAcquiredBufferCount);
+ return mBbq.mBufferReleaseConsumer->readReleaseFence(outId, outFence,
+ outMaxAcquiredBufferCount);
}
void BLASTBufferQueue::BufferReleaseReader::interruptBlockingRead() {
- uint64_t value = 1;
- if (write(mEventFd.get(), &value, sizeof(uint64_t)) == -1) {
+ if (eventfd_write(mEventFd.get(), 1) == -1) {
ALOGE("failed to notify dequeue event. errno=%d message='%s'", errno, strerror(errno));
}
}
-void BLASTBufferQueue::BufferReleaseThread::start(const sp<BLASTBufferQueue>& bbq) {
- mRunning = std::make_shared<std::atomic_bool>(true);
- mReader = bbq->mBufferReleaseReader;
- std::thread([running = mRunning, reader = mReader, weakBbq = wp<BLASTBufferQueue>(bbq)]() {
+void BLASTBufferQueue::BufferReleaseReader::clearInterrupts() {
+ eventfd_t value;
+ if (eventfd_read(mEventFd.get(), &value) == -1 && errno != EWOULDBLOCK) {
+ ALOGE("error while reading from eventfd. errno=%d message='%s'", errno, strerror(errno));
+ }
+}
+
+BLASTBufferQueue::BufferReleaseThread::BufferReleaseThread(const sp<BLASTBufferQueue>& bbq) {
+ android::base::unique_fd epollFd{epoll_create1(EPOLL_CLOEXEC)};
+ LOG_ALWAYS_FATAL_IF(!epollFd.ok(),
+ "Failed to create buffer release background thread epoll file descriptor. "
+ "errno=%d message='%s'",
+ errno, strerror(errno));
+
+ epoll_event registerEndpointFd{};
+ registerEndpointFd.events = EPOLLIN;
+ registerEndpointFd.data.fd = bbq->mBufferReleaseConsumer->getFd();
+ status_t status = epoll_ctl(epollFd.get(), EPOLL_CTL_ADD, bbq->mBufferReleaseConsumer->getFd(),
+ ®isterEndpointFd);
+ LOG_ALWAYS_FATAL_IF(status == -1,
+ "Failed to register background thread buffer release consumer file "
+ "descriptor with epoll. errno=%d message='%s'",
+ errno, strerror(errno));
+
+ // EventFd is used to break the background thread's loop.
+ android::base::unique_fd eventFd{eventfd(0, EFD_CLOEXEC | EFD_NONBLOCK)};
+ LOG_ALWAYS_FATAL_IF(!eventFd.ok(),
+ "Failed to create background thread buffer release event file descriptor. "
+ "errno=%d message='%s'",
+ errno, strerror(errno));
+
+ epoll_event registerEventFd{};
+ registerEventFd.events = EPOLLIN;
+ registerEventFd.data.fd = eventFd.get();
+ status = epoll_ctl(epollFd.get(), EPOLL_CTL_ADD, eventFd.get(), ®isterEventFd);
+ LOG_ALWAYS_FATAL_IF(status == -1,
+ "Failed to register background thread event file descriptor with epoll. "
+ "errno=%d message='%s'",
+ errno, strerror(errno));
+
+ mEventFd = eventFd.get();
+
+ std::thread([epollFd = std::move(epollFd), eventFd = std::move(eventFd),
+ weakBbq = wp<BLASTBufferQueue>(bbq)]() {
pthread_setname_np(pthread_self(), "BufferReleaseThread");
- while (*running) {
- ReleaseCallbackId id;
- sp<Fence> fence;
- uint32_t maxAcquiredBufferCount;
- if (status_t status = reader->readBlocking(id, fence, maxAcquiredBufferCount);
- status != OK) {
+ while (true) {
+ epoll_event event{};
+ int eventCount;
+ do {
+ eventCount = epoll_wait(epollFd.get(), &event, 1 /*maxevents*/, -1 /*timeout*/);
+ } while (eventCount == -1 && errno != EINTR);
+
+ if (eventCount == -1) {
+ ALOGE("epoll_wait error while waiting for buffer release in background thread. "
+ "errno=%d message='%s'",
+ errno, strerror(errno));
continue;
}
+
+ // EventFd is used to join this thread.
+ if (event.data.fd == eventFd.get()) {
+ return;
+ }
+
sp<BLASTBufferQueue> bbq = weakBbq.promote();
if (!bbq) {
return;
}
+
+ // If there are threads blocking on dequeue, give those threads priority for handling
+ // the release.
+ if (bbq->mThreadsBlockingOnDequeue > 0) {
+ std::this_thread::sleep_for(0ms);
+ continue;
+ }
+
+ ReleaseCallbackId id;
+ sp<Fence> fence;
+ uint32_t maxAcquiredBufferCount;
+ status_t status = bbq->mBufferReleaseConsumer->readReleaseFence(id, fence,
+ maxAcquiredBufferCount);
+ if (status != OK) {
+ ALOGE("failed to read from buffer release consumer in background thread. errno=%d "
+ "message='%s'",
+ errno, strerror(errno));
+ continue;
+ }
bbq->releaseBufferCallback(id, fence, maxAcquiredBufferCount);
}
}).detach();
}
BLASTBufferQueue::BufferReleaseThread::~BufferReleaseThread() {
- *mRunning = false;
- mReader->interruptBlockingRead();
+ eventfd_write(mEventFd, 1);
}
#endif
diff --git a/libs/gui/BufferQueueConsumer.cpp b/libs/gui/BufferQueueConsumer.cpp
index 69d25be..d0607bf 100644
--- a/libs/gui/BufferQueueConsumer.cpp
+++ b/libs/gui/BufferQueueConsumer.cpp
@@ -297,7 +297,11 @@
// We might have freed a slot while dropping old buffers, or the producer
// may be blocked waiting for the number of buffers in the queue to
// decrease.
+#if COM_ANDROID_GRAPHICS_LIBGUI_FLAGS(BUFFER_RELEASE_CHANNEL)
+ mCore->notifyBufferReleased();
+#else
mCore->mDequeueCondition.notify_all();
+#endif
ATRACE_INT(mCore->mConsumerName.c_str(), static_cast<int32_t>(mCore->mQueue.size()));
#ifndef NO_BINDER
@@ -350,7 +354,12 @@
mCore->mActiveBuffers.erase(slot);
mCore->mFreeSlots.insert(slot);
mCore->clearBufferSlotLocked(slot);
+#if COM_ANDROID_GRAPHICS_LIBGUI_FLAGS(BUFFER_RELEASE_CHANNEL)
+ mCore->notifyBufferReleased();
+#else
mCore->mDequeueCondition.notify_all();
+#endif
+
VALIDATE_CONSISTENCY();
}
@@ -520,7 +529,12 @@
}
BQ_LOGV("releaseBuffer: releasing slot %d", slot);
+#if COM_ANDROID_GRAPHICS_LIBGUI_FLAGS(BUFFER_RELEASE_CHANNEL)
+ mCore->notifyBufferReleased();
+#else
mCore->mDequeueCondition.notify_all();
+#endif
+
VALIDATE_CONSISTENCY();
} // Autolock scope
@@ -574,7 +588,11 @@
mCore->mQueue.clear();
mCore->freeAllBuffersLocked();
mCore->mSharedBufferSlot = BufferQueueCore::INVALID_BUFFER_SLOT;
+#if COM_ANDROID_GRAPHICS_LIBGUI_FLAGS(BUFFER_RELEASE_CHANNEL)
+ mCore->notifyBufferReleased();
+#else
mCore->mDequeueCondition.notify_all();
+#endif
return NO_ERROR;
}
diff --git a/libs/gui/BufferQueueCore.cpp b/libs/gui/BufferQueueCore.cpp
index e0c5b1f..d52cf70 100644
--- a/libs/gui/BufferQueueCore.cpp
+++ b/libs/gui/BufferQueueCore.cpp
@@ -371,6 +371,12 @@
}
}
+#if COM_ANDROID_GRAPHICS_LIBGUI_FLAGS(BUFFER_RELEASE_CHANNEL)
+void BufferQueueCore::notifyBufferReleased() const {
+ mDequeueCondition.notify_all();
+}
+#endif
+
#if DEBUG_ONLY_CODE
void BufferQueueCore::validateConsistencyLocked() const {
static const useconds_t PAUSE_TIME = 0;
diff --git a/libs/gui/BufferQueueProducer.cpp b/libs/gui/BufferQueueProducer.cpp
index da74e9c..e58233c 100644
--- a/libs/gui/BufferQueueProducer.cpp
+++ b/libs/gui/BufferQueueProducer.cpp
@@ -202,7 +202,11 @@
if (delta < 0) {
listener = mCore->mConsumerListener;
}
+#if COM_ANDROID_GRAPHICS_LIBGUI_FLAGS(BUFFER_RELEASE_CHANNEL)
+ mCore->notifyBufferReleased();
+#else
mCore->mDequeueCondition.notify_all();
+#endif
} // Autolock scope
// Call back without lock held
@@ -254,7 +258,12 @@
}
mCore->mAsyncMode = async;
VALIDATE_CONSISTENCY();
+#if COM_ANDROID_GRAPHICS_LIBGUI_FLAGS(BUFFER_RELEASE_CHANNEL)
+ mCore->notifyBufferReleased();
+#else
mCore->mDequeueCondition.notify_all();
+#endif
+
if (delta < 0) {
listener = mCore->mConsumerListener;
}
@@ -376,6 +385,12 @@
(acquiredCount <= mCore->mMaxAcquiredBufferCount)) {
return WOULD_BLOCK;
}
+#if COM_ANDROID_GRAPHICS_LIBGUI_FLAGS(BUFFER_RELEASE_CHANNEL)
+ if (status_t status = waitForBufferRelease(lock, mDequeueTimeout);
+ status == TIMED_OUT) {
+ return TIMED_OUT;
+ }
+#else
if (mDequeueTimeout >= 0) {
std::cv_status result = mCore->mDequeueCondition.wait_for(lock,
std::chrono::nanoseconds(mDequeueTimeout));
@@ -385,12 +400,29 @@
} else {
mCore->mDequeueCondition.wait(lock);
}
+#endif
}
} // while (tryAgain)
return NO_ERROR;
}
+#if COM_ANDROID_GRAPHICS_LIBGUI_FLAGS(BUFFER_RELEASE_CHANNEL)
+status_t BufferQueueProducer::waitForBufferRelease(std::unique_lock<std::mutex>& lock,
+ nsecs_t timeout) const {
+ if (mDequeueTimeout >= 0) {
+ std::cv_status result =
+ mCore->mDequeueCondition.wait_for(lock, std::chrono::nanoseconds(timeout));
+ if (result == std::cv_status::timeout) {
+ return TIMED_OUT;
+ }
+ } else {
+ mCore->mDequeueCondition.wait(lock);
+ }
+ return OK;
+}
+#endif
+
status_t BufferQueueProducer::dequeueBuffer(int* outSlot, sp<android::Fence>* outFence,
uint32_t width, uint32_t height, PixelFormat format,
uint64_t usage, uint64_t* outBufferAge,
@@ -741,7 +773,11 @@
mCore->mActiveBuffers.erase(slot);
mCore->mFreeSlots.insert(slot);
mCore->clearBufferSlotLocked(slot);
+#if COM_ANDROID_GRAPHICS_LIBGUI_FLAGS(BUFFER_RELEASE_CHANNEL)
+ mCore->notifyBufferReleased();
+#else
mCore->mDequeueCondition.notify_all();
+#endif
VALIDATE_CONSISTENCY();
}
@@ -1082,7 +1118,11 @@
}
mCore->mBufferHasBeenQueued = true;
+#if COM_ANDROID_GRAPHICS_LIBGUI_FLAGS(BUFFER_RELEASE_CHANNEL)
+ mCore->notifyBufferReleased();
+#else
mCore->mDequeueCondition.notify_all();
+#endif
mCore->mLastQueuedSlot = slot;
output->width = mCore->mDefaultWidth;
@@ -1218,7 +1258,11 @@
bufferId = gb->getId();
}
mSlots[slot].mFence = fence;
+#if COM_ANDROID_GRAPHICS_LIBGUI_FLAGS(BUFFER_RELEASE_CHANNEL)
+ mCore->notifyBufferReleased();
+#else
mCore->mDequeueCondition.notify_all();
+#endif
listener = mCore->mConsumerListener;
VALIDATE_CONSISTENCY();
}
@@ -1457,7 +1501,11 @@
mCore->mConnectedApi = BufferQueueCore::NO_CONNECTED_API;
mCore->mConnectedPid = -1;
mCore->mSidebandStream.clear();
+#if COM_ANDROID_GRAPHICS_LIBGUI_FLAGS(BUFFER_RELEASE_CHANNEL)
+ mCore->notifyBufferReleased();
+#else
mCore->mDequeueCondition.notify_all();
+#endif
mCore->mAutoPrerotation = false;
#if COM_ANDROID_GRAPHICS_LIBGUI_FLAGS(BQ_EXTENDEDALLOCATE)
mCore->mAdditionalOptions.clear();
diff --git a/libs/gui/BufferReleaseChannel.cpp b/libs/gui/BufferReleaseChannel.cpp
index 27367aa..e9c6ef3 100644
--- a/libs/gui/BufferReleaseChannel.cpp
+++ b/libs/gui/BufferReleaseChannel.cpp
@@ -136,6 +136,7 @@
status_t BufferReleaseChannel::ConsumerEndpoint::readReleaseFence(
ReleaseCallbackId& outReleaseCallbackId, sp<Fence>& outReleaseFence,
uint32_t& outMaxAcquiredBufferCount) {
+ std::lock_guard lock{mMutex};
Message message;
mFlattenedBuffer.resize(message.getFlattenedSize());
std::array<uint8_t, CMSG_SPACE(sizeof(int))> controlMessageBuffer;
@@ -152,7 +153,7 @@
.msg_controllen = controlMessageBuffer.size(),
};
- int result;
+ ssize_t result;
do {
result = recvmsg(mFd, &msg, 0);
} while (result == -1 && errno == EINTR);
@@ -242,7 +243,7 @@
memcpy(CMSG_DATA(cmsg), &flattenedFd, sizeof(int));
}
- int result;
+ ssize_t result;
do {
result = sendmsg(mFd, &msg, 0);
} while (result == -1 && errno == EINTR);
diff --git a/libs/gui/include/gui/BLASTBufferQueue.h b/libs/gui/include/gui/BLASTBufferQueue.h
index 8592cff..99c64da 100644
--- a/libs/gui/include/gui/BLASTBufferQueue.h
+++ b/libs/gui/include/gui/BLASTBufferQueue.h
@@ -150,6 +150,9 @@
private:
friend class BLASTBufferQueueHelper;
friend class BBQBufferQueueProducer;
+#if COM_ANDROID_GRAPHICS_LIBGUI_FLAGS(BUFFER_RELEASE_CHANNEL)
+ friend class BBQBufferQueueCore;
+#endif
// can't be copied
BLASTBufferQueue& operator = (const BLASTBufferQueue& rhs);
@@ -317,48 +320,52 @@
std::unordered_set<uint64_t> mSyncedFrameNumbers GUARDED_BY(mMutex);
#if COM_ANDROID_GRAPHICS_LIBGUI_FLAGS(BUFFER_RELEASE_CHANNEL)
+ // BufferReleaseChannel is used to communicate buffer releases from SurfaceFlinger to the
+ // client.
+ std::unique_ptr<gui::BufferReleaseChannel::ConsumerEndpoint> mBufferReleaseConsumer;
+ std::shared_ptr<gui::BufferReleaseChannel::ProducerEndpoint> mBufferReleaseProducer;
+
class BufferReleaseReader {
public:
- BufferReleaseReader() = default;
- BufferReleaseReader(std::unique_ptr<gui::BufferReleaseChannel::ConsumerEndpoint>);
- BufferReleaseReader& operator=(BufferReleaseReader&&);
+ explicit BufferReleaseReader(BLASTBufferQueue&);
+
+ BufferReleaseReader(const BufferReleaseReader&) = delete;
+ BufferReleaseReader& operator=(const BufferReleaseReader&) = delete;
// Block until we can read a buffer release message.
//
// Returns:
// * OK if a ReleaseCallbackId and Fence were successfully read.
// * WOULD_BLOCK if the blocking read was interrupted by interruptBlockingRead.
+ // * TIMED_OUT if the blocking read timed out.
// * UNKNOWN_ERROR if something went wrong.
status_t readBlocking(ReleaseCallbackId& outId, sp<Fence>& outReleaseFence,
- uint32_t& outMaxAcquiredBufferCount);
+ uint32_t& outMaxAcquiredBufferCount, nsecs_t timeout);
- // Signals the reader's eventfd to wake up any threads waiting on readBlocking.
void interruptBlockingRead();
+ void clearInterrupts();
private:
- std::mutex mMutex;
- std::unique_ptr<gui::BufferReleaseChannel::ConsumerEndpoint> mEndpoint GUARDED_BY(mMutex);
+ BLASTBufferQueue& mBbq;
+
android::base::unique_fd mEpollFd;
android::base::unique_fd mEventFd;
};
- // BufferReleaseChannel is used to communicate buffer releases from SurfaceFlinger to
- // the client. See BBQBufferQueueProducer::dequeueBuffer for details.
- std::shared_ptr<BufferReleaseReader> mBufferReleaseReader;
- std::shared_ptr<gui::BufferReleaseChannel::ProducerEndpoint> mBufferReleaseProducer;
+ std::optional<BufferReleaseReader> mBufferReleaseReader;
+
+ std::atomic<int> mThreadsBlockingOnDequeue = 0;
class BufferReleaseThread {
public:
- BufferReleaseThread() = default;
+ BufferReleaseThread(const sp<BLASTBufferQueue>&);
~BufferReleaseThread();
- void start(const sp<BLASTBufferQueue>&);
private:
- std::shared_ptr<std::atomic_bool> mRunning;
- std::shared_ptr<BufferReleaseReader> mReader;
+ int mEventFd;
};
- BufferReleaseThread mBufferReleaseThread;
+ std::optional<BufferReleaseThread> mBufferReleaseThread;
#endif
};
diff --git a/libs/gui/include/gui/BufferQueueCore.h b/libs/gui/include/gui/BufferQueueCore.h
index d5dd7c8..77cdf2c 100644
--- a/libs/gui/include/gui/BufferQueueCore.h
+++ b/libs/gui/include/gui/BufferQueueCore.h
@@ -80,6 +80,13 @@
BufferQueueCore();
virtual ~BufferQueueCore();
+#if COM_ANDROID_GRAPHICS_LIBGUI_FLAGS(BUFFER_RELEASE_CHANNEL)
+protected:
+ // Wake up any threads waiting for a buffer release. The BufferQueue mutex should always held
+ // when this method is called.
+ virtual void notifyBufferReleased() const;
+#endif
+
private:
// Dump our state in a string
void dumpState(const String8& prefix, String8* outResult) const;
diff --git a/libs/gui/include/gui/BufferQueueProducer.h b/libs/gui/include/gui/BufferQueueProducer.h
index 37a9607..086ce7c 100644
--- a/libs/gui/include/gui/BufferQueueProducer.h
+++ b/libs/gui/include/gui/BufferQueueProducer.h
@@ -218,6 +218,14 @@
// total maximum buffer count for the buffer queue (dequeued AND acquired)
status_t setMaxDequeuedBufferCount(int maxDequeuedBuffers, int* maxBufferCount);
+#if COM_ANDROID_GRAPHICS_LIBGUI_FLAGS(BUFFER_RELEASE_CHANNEL)
+ // Wait until a buffer has been released. The method may spuriously return OK when no buffer has
+ // been released. The BufferQueue mutex is passed in the locked state. It must be unlocked
+ // before waiting for a release and locked before returning.
+ virtual status_t waitForBufferRelease(std::unique_lock<std::mutex>& lock,
+ nsecs_t timeout) const;
+#endif
+
private:
// This is required by the IBinder::DeathRecipient interface
virtual void binderDied(const wp<IBinder>& who);
diff --git a/libs/gui/include/gui/BufferReleaseChannel.h b/libs/gui/include/gui/BufferReleaseChannel.h
index 51fe0b6..0edadec 100644
--- a/libs/gui/include/gui/BufferReleaseChannel.h
+++ b/libs/gui/include/gui/BufferReleaseChannel.h
@@ -69,7 +69,8 @@
sp<Fence>& outReleaseFence, uint32_t& maxAcquiredBufferCount);
private:
- std::vector<uint8_t> mFlattenedBuffer;
+ std::mutex mMutex;
+ std::vector<uint8_t> mFlattenedBuffer GUARDED_BY(mMutex);
};
class ProducerEndpoint : public Endpoint, public Parcelable {