Refactor VrFlinger to use BufferHubQueue

1/ Remove DisplayRPC::AllocateBuffer, as individual buffer allocation
is now handled by BufferHubQueue.
2/ Reimplement native_buffer_queue using bufferhubqueue.
3/ Hook up consumer queue in DisplaySurface.
4/ Remove epoll_event_dispatcher as its no longer being used.

Bug: 36033302
Test: Built and ran particles.apk
Change-Id: I38ee1c57195888ede935ebc50119bcb7e4ab4e36
diff --git a/libs/vr/libbufferhubqueue/Android.bp b/libs/vr/libbufferhubqueue/Android.bp
index 10198fd..b976097 100644
--- a/libs/vr/libbufferhubqueue/Android.bp
+++ b/libs/vr/libbufferhubqueue/Android.bp
@@ -45,6 +45,7 @@
     cflags = [ "-DLOGTAG=\"libbufferhubqueue\"" ],
     srcs: sourceFiles,
     export_include_dirs: includeFiles,
+    export_static_lib_headers: staticLibraries,
     static_libs: staticLibraries,
     shared_libs: sharedLibraries,
 }
diff --git a/libs/vr/libdisplay/display_client.cpp b/libs/vr/libdisplay/display_client.cpp
index 54098e8..dcdd994 100644
--- a/libs/vr/libdisplay/display_client.cpp
+++ b/libs/vr/libdisplay/display_client.cpp
@@ -148,18 +148,22 @@
   }
 }
 
-std::shared_ptr<BufferProducer> DisplaySurfaceClient::AllocateBuffer(
-    uint32_t* buffer_index) {
-  auto status = InvokeRemoteMethod<DisplayRPC::AllocateBuffer>();
-  if (!status) {
-    ALOGE("DisplaySurfaceClient::AllocateBuffer: Failed to allocate buffer: %s",
+std::shared_ptr<ProducerQueue> DisplaySurfaceClient::GetProducerQueue() {
+  if (producer_queue_ == nullptr) {
+    // Create producer queue through DisplayRPC
+    auto status = InvokeRemoteMethod<DisplayRPC::CreateBufferQueue>();
+    if (!status) {
+      ALOGE(
+          "DisplaySurfaceClient::GetProducerQueue: failed to create producer "
+          "queue: %s",
           status.GetErrorMessage().c_str());
-    return nullptr;
-  }
+      return nullptr;
+    }
 
-  if (buffer_index)
-    *buffer_index = status.get().first;
-  return BufferProducer::Import(status.take().second);
+    producer_queue_ =
+        ProducerQueue::Import<DisplaySurfaceMetadata>(status.take());
+  }
+  return producer_queue_;
 }
 
 volatile DisplaySurfaceMetadata* DisplaySurfaceClient::GetMetadataBufferPtr() {
diff --git a/libs/vr/libdisplay/include/private/dvr/display_client.h b/libs/vr/libdisplay/include/private/dvr/display_client.h
index 034b7b4..e1471c3 100644
--- a/libs/vr/libdisplay/include/private/dvr/display_client.h
+++ b/libs/vr/libdisplay/include/private/dvr/display_client.h
@@ -5,6 +5,7 @@
 #include <pdx/client.h>
 #include <pdx/file_handle.h>
 #include <private/dvr/buffer_hub_client.h>
+#include <private/dvr/buffer_hub_queue_client.h>
 #include <private/dvr/display_rpc.h>
 
 namespace android {
@@ -62,13 +63,9 @@
   void SetBlurBehind(bool blur_behind);
   void SetAttributes(const DisplaySurfaceAttributes& attributes);
 
-  // |out_buffer_index| will receive a unique index for this buffer within the
-  // surface. The first buffer gets 0, second gets 1, and so on. This index
-  // can be used to deliver metadata for buffers that are queued for display.
-  std::shared_ptr<BufferProducer> AllocateBuffer(uint32_t* out_buffer_index);
-  std::shared_ptr<BufferProducer> AllocateBuffer() {
-    return AllocateBuffer(nullptr);
-  }
+  // Get the producer end of the buffer queue that transports graphics buffer
+  // from the application side to the compositor side.
+  std::shared_ptr<ProducerQueue> GetProducerQueue();
 
   // Get the shared memory metadata buffer for this display surface. If it is
   // not yet allocated, this will allocate it.
@@ -93,6 +90,9 @@
   bool blur_behind_;
   DisplaySurfaceMetadata* mapped_metadata_buffer_;
 
+  // TODO(jwcai) Add support for multiple queues.
+  std::shared_ptr<ProducerQueue> producer_queue_;
+
   DisplaySurfaceClient(const DisplaySurfaceClient&) = delete;
   void operator=(const DisplaySurfaceClient&) = delete;
 };
diff --git a/libs/vr/libdisplay/include/private/dvr/display_rpc.h b/libs/vr/libdisplay/include/private/dvr/display_rpc.h
index d37aed7..465fbae 100644
--- a/libs/vr/libdisplay/include/private/dvr/display_rpc.h
+++ b/libs/vr/libdisplay/include/private/dvr/display_rpc.h
@@ -212,7 +212,7 @@
     kOpGetMetrics = 0,
     kOpGetEdsCapture,
     kOpCreateSurface,
-    kOpAllocateBuffer,
+    kOpCreateBufferQueue,
     kOpSetAttributes,
     kOpGetMetadataBuffer,
     kOpCreateVideoMeshSurface,
@@ -233,8 +233,8 @@
   PDX_REMOTE_METHOD(CreateSurface, kOpCreateSurface,
                     int(int width, int height, int format, int usage,
                         DisplaySurfaceFlags flags));
-  PDX_REMOTE_METHOD(AllocateBuffer, kOpAllocateBuffer,
-                    std::pair<std::uint32_t, LocalChannelHandle>(Void));
+  PDX_REMOTE_METHOD(CreateBufferQueue, kOpCreateBufferQueue,
+                    LocalChannelHandle(Void));
   PDX_REMOTE_METHOD(SetAttributes, kOpSetAttributes,
                     int(const DisplaySurfaceAttributes& attributes));
   PDX_REMOTE_METHOD(GetMetadataBuffer, kOpGetMetadataBuffer,
diff --git a/libs/vr/libdisplay/include/private/dvr/native_buffer_queue.h b/libs/vr/libdisplay/include/private/dvr/native_buffer_queue.h
index 87e9c9f..4b1fa98 100644
--- a/libs/vr/libdisplay/include/private/dvr/native_buffer_queue.h
+++ b/libs/vr/libdisplay/include/private/dvr/native_buffer_queue.h
@@ -14,57 +14,27 @@
 namespace android {
 namespace dvr {
 
-// NativeBufferQueue manages a queue of NativeBufferProducers allocated from a
-// DisplaySurfaceClient. Buffers are automatically re-enqueued when released by
-// the consumer side.
+// A wrapper over dvr::ProducerQueue that caches EGLImage.
 class NativeBufferQueue {
  public:
   // Create a queue with the given number of free buffers.
-  NativeBufferQueue(const std::shared_ptr<DisplaySurfaceClient>& surface,
-                    size_t capacity);
   NativeBufferQueue(EGLDisplay display,
                     const std::shared_ptr<DisplaySurfaceClient>& surface,
                     size_t capacity);
-  ~NativeBufferQueue();
 
-  std::shared_ptr<DisplaySurfaceClient> surface() const { return surface_; }
+  size_t GetQueueCapacity() const { return producer_queue_->capacity(); }
 
   // Dequeue a buffer from the free queue, blocking until one is available.
   NativeBufferProducer* Dequeue();
 
-  // Enqueue a buffer at the end of the free queue.
-  void Enqueue(NativeBufferProducer* buf);
-
-  // Get the number of free buffers in the queue.
-  size_t GetFreeBufferCount() const;
-
-  // Get the total number of buffers managed by this queue.
-  size_t GetQueueCapacity() const;
-
-  // Accessors for display surface buffer attributes.
-  int width() const { return surface_->width(); }
-  int height() const { return surface_->height(); }
-  int format() const { return surface_->format(); }
-  int usage() const { return surface_->usage(); }
+  // An noop here to keep Vulkan path in GraphicsContext happy.
+  // TODO(jwcai, cort) Move Vulkan path into GVR/Google3.
+  void Enqueue(NativeBufferProducer* buffer) {}
 
  private:
-  // Wait for buffers to be released and enqueue them.
-  bool WaitForBuffers();
-
-  std::shared_ptr<DisplaySurfaceClient> surface_;
-
-  // A list of strong pointers to the buffers, used for managing buffer
-  // lifetime.
-  std::vector<android::sp<NativeBufferProducer>> buffers_;
-
-  // Used to implement queue semantics.
-  RingBuffer<NativeBufferProducer*> buffer_queue_;
-
-  // Epoll fd used to wait for BufferHub events.
-  int epoll_fd_;
-
-  NativeBufferQueue(const NativeBufferQueue&) = delete;
-  void operator=(NativeBufferQueue&) = delete;
+  EGLDisplay display_;
+  std::shared_ptr<ProducerQueue> producer_queue_;
+  std::vector<sp<NativeBufferProducer>> buffers_;
 };
 
 }  // namespace dvr
diff --git a/libs/vr/libdisplay/include/private/dvr/video_mesh_surface_client.h b/libs/vr/libdisplay/include/private/dvr/video_mesh_surface_client.h
index e52d0b9..3a7f125 100644
--- a/libs/vr/libdisplay/include/private/dvr/video_mesh_surface_client.h
+++ b/libs/vr/libdisplay/include/private/dvr/video_mesh_surface_client.h
@@ -25,7 +25,7 @@
  private:
   friend BASE;
 
-  std::shared_ptr<android::dvr::ProducerQueue> producer_queue_;
+  std::shared_ptr<ProducerQueue> producer_queue_;
   VideoMeshSurfaceMetadata* mapped_metadata_buffer_;
 
   explicit VideoMeshSurfaceClient(LocalChannelHandle handle);
diff --git a/libs/vr/libdisplay/native_buffer_queue.cpp b/libs/vr/libdisplay/native_buffer_queue.cpp
index 8dd0ee0..d516d63 100644
--- a/libs/vr/libdisplay/native_buffer_queue.cpp
+++ b/libs/vr/libdisplay/native_buffer_queue.cpp
@@ -13,138 +13,51 @@
 namespace dvr {
 
 NativeBufferQueue::NativeBufferQueue(
-    const std::shared_ptr<DisplaySurfaceClient>& surface, size_t capacity)
-    : NativeBufferQueue(nullptr, surface, capacity) {}
-
-NativeBufferQueue::NativeBufferQueue(
     EGLDisplay display, const std::shared_ptr<DisplaySurfaceClient>& surface,
     size_t capacity)
-    : surface_(surface),
-      buffers_(capacity),
-      buffer_queue_(capacity) {
-  LOG_ALWAYS_FATAL_IF(!surface);
-
-  epoll_fd_ = epoll_create(64);
-  if (epoll_fd_ < 0) {
-    ALOGE("NativeBufferQueue::NativeBufferQueue: Failed to create epoll fd: %s",
-          strerror(errno));
-    return;
-  }
-
-  // The kSurfaceBufferMaxCount must be >= the capacity so that shader code
-  // can bind surface buffer array data.
-  LOG_ALWAYS_FATAL_IF(kSurfaceBufferMaxCount < capacity);
+    : display_(display), buffers_(capacity) {
+  std::shared_ptr<ProducerQueue> queue = surface->GetProducerQueue();
 
   for (size_t i = 0; i < capacity; i++) {
-    uint32_t buffer_index = 0;
-    auto buffer = surface_->AllocateBuffer(&buffer_index);
-    if (!buffer) {
-      ALOGE("NativeBufferQueue::NativeBufferQueue: Failed to allocate buffer!");
-      return;
-    }
-
-    // TODO(jbates): store an index associated with each buffer so that we can
-    // determine which index in DisplaySurfaceMetadata it is associated
-    // with.
-    buffers_.push_back(new NativeBufferProducer(buffer, display, buffer_index));
-    NativeBufferProducer* native_buffer = buffers_.back().get();
-
-    epoll_event event = {.events = EPOLLIN | EPOLLET,
-                         .data = {.ptr = native_buffer}};
-    if (epoll_ctl(epoll_fd_, EPOLL_CTL_ADD, buffer->event_fd(), &event) <
-        0) {
+    size_t slot;
+    // TODO(jwcai) Should change to use BufferViewPort's spec to config.
+    int ret =
+        queue->AllocateBuffer(surface->width(), surface->height(),
+                              surface->format(), surface->usage(), 1, &slot);
+    if (ret < 0) {
       ALOGE(
-          "NativeBufferQueue::NativeBufferQueue: Failed to add buffer producer "
-          "to epoll set: %s",
-          strerror(errno));
+          "NativeBufferQueue::NativeBufferQueue: Failed to allocate buffer, "
+          "error=%d",
+          ret);
       return;
     }
 
-    Enqueue(native_buffer);
-  }
-}
-
-NativeBufferQueue::~NativeBufferQueue() {
-  if (epoll_fd_ >= 0)
-    close(epoll_fd_);
-}
-
-bool NativeBufferQueue::WaitForBuffers() {
-  ATRACE_NAME("NativeBufferQueue::WaitForBuffers");
-  // Intentionally set this to one so that we don't waste time retrieving too
-  // many buffers.
-  constexpr size_t kMaxEvents = 1;
-  std::array<epoll_event, kMaxEvents> events;
-
-  while (buffer_queue_.IsEmpty()) {
-    int num_events = epoll_wait(epoll_fd_, events.data(), events.size(), -1);
-    if (num_events < 0 && errno != EINTR) {
-      ALOGE("NativeBufferQueue:WaitForBuffers: Failed to wait for buffers: %s",
-            strerror(errno));
-      return false;
-    }
-
-    ALOGD_IF(TRACE, "NativeBufferQueue::WaitForBuffers: num_events=%d",
-             num_events);
-
-    for (int i = 0; i < num_events; i++) {
-      NativeBufferProducer* buffer =
-          static_cast<NativeBufferProducer*>(events[i].data.ptr);
-      ALOGD_IF(TRACE,
-               "NativeBufferQueue::WaitForBuffers: event %d: buffer_id=%d "
-               "events=0x%x",
-               i, buffer->buffer()->id(), events[i].events);
-
-      if (events[i].events & EPOLLIN) {
-        const int ret = buffer->GainAsync();
-        if (ret < 0) {
-          ALOGE("NativeBufferQueue::WaitForBuffers: Failed to gain buffer: %s",
-                strerror(-ret));
-          continue;
-        }
-
-        Enqueue(buffer);
-      }
-    }
+    ALOGD_IF(TRACE,
+             "NativeBufferQueue::NativeBufferQueue: New buffer allocated at "
+             "slot=%zu",
+             slot);
   }
 
-  return true;
-}
-
-void NativeBufferQueue::Enqueue(NativeBufferProducer* buf) {
-  ATRACE_NAME("NativeBufferQueue::Enqueue");
-  if (buffer_queue_.IsFull()) {
-    ALOGE("NativeBufferQueue::Enqueue: Queue is full!");
-    return;
-  }
-
-  buffer_queue_.Append(buf);
+  producer_queue_ = std::move(queue);
 }
 
 NativeBufferProducer* NativeBufferQueue::Dequeue() {
   ATRACE_NAME("NativeBufferQueue::Dequeue");
-  ALOGD_IF(TRACE, "NativeBufferQueue::Dequeue: count=%zd",
-           buffer_queue_.GetSize());
 
-  if (buffer_queue_.IsEmpty() && !WaitForBuffers())
-    return nullptr;
+  // This never times out.
+  size_t slot;
+  pdx::LocalHandle fence;
+  std::shared_ptr<BufferProducer> buffer =
+      producer_queue_->Dequeue(-1, &slot, &fence);
 
-  NativeBufferProducer* buf = buffer_queue_.Front();
-  buffer_queue_.PopFront();
-  if (buf == nullptr) {
-    ALOGE("NativeBufferQueue::Dequeue: Buffer at tail was nullptr!!!");
-    return nullptr;
+  if (buffers_[slot] == nullptr) {
+    buffers_[slot] = new NativeBufferProducer(buffer, display_, slot);
   }
 
-  return buf;
-}
-
-size_t NativeBufferQueue::GetFreeBufferCount() const {
-  return buffer_queue_.GetSize();
-}
-
-size_t NativeBufferQueue::GetQueueCapacity() const {
-  return buffer_queue_.GetCapacity();
+  ALOGD_IF(TRACE,
+           "NativeBufferQueue::Dequeue: dequeue buffer at slot=%zu, buffer=%p",
+           slot, buffers_[slot].get());
+  return buffers_[slot].get();
 }
 
 }  // namespace dvr
diff --git a/libs/vr/libeds/Android.bp b/libs/vr/libeds/Android.bp
index a9df847..7838c4d 100644
--- a/libs/vr/libeds/Android.bp
+++ b/libs/vr/libeds/Android.bp
@@ -79,7 +79,8 @@
         "libgmock",
         "libeds",
     ] + staticLibraries + [
-        "libbufferhub"
+        "libbufferhub",
+        "libbufferhubqueue",
     ],
 
 }
diff --git a/libs/vr/libvrflinger/Android.bp b/libs/vr/libvrflinger/Android.bp
index 4ad7c23..3f79a7b 100644
--- a/libs/vr/libvrflinger/Android.bp
+++ b/libs/vr/libvrflinger/Android.bp
@@ -20,7 +20,6 @@
     "display_manager_service.cpp",
     "display_service.cpp",
     "display_surface.cpp",
-    "epoll_event_dispatcher.cpp",
     "hardware_composer.cpp",
     "screenshot_service.cpp",
     "surface_channel.cpp",
diff --git a/libs/vr/libvrflinger/display_service.cpp b/libs/vr/libvrflinger/display_service.cpp
index bb70c5c..fdb84c5 100644
--- a/libs/vr/libvrflinger/display_service.cpp
+++ b/libs/vr/libvrflinger/display_service.cpp
@@ -90,7 +90,7 @@
       return 0;
 
     // Direct the surface specific messages to the surface instance.
-    case DisplayRPC::AllocateBuffer::Opcode:
+    case DisplayRPC::CreateBufferQueue::Opcode:
     case DisplayRPC::SetAttributes::Opcode:
     case DisplayRPC::GetMetadataBuffer::Opcode:
     case DisplayRPC::CreateVideoMeshSurface::Opcode:
diff --git a/libs/vr/libvrflinger/display_service.h b/libs/vr/libvrflinger/display_service.h
index b207e4d..9d116c1 100644
--- a/libs/vr/libvrflinger/display_service.h
+++ b/libs/vr/libvrflinger/display_service.h
@@ -14,7 +14,6 @@
 
 #include "acquired_buffer.h"
 #include "display_surface.h"
-#include "epoll_event_dispatcher.h"
 #include "hardware_composer.h"
 
 namespace android {
@@ -99,7 +98,6 @@
   DisplayService(const DisplayService&) = delete;
   void operator=(const DisplayService&) = delete;
 
-  EpollEventDispatcher dispatcher_;
   HardwareComposer hardware_composer_;
   DisplayConfigurationUpdateNotifier update_notifier_;
 };
diff --git a/libs/vr/libvrflinger/display_surface.cpp b/libs/vr/libvrflinger/display_surface.cpp
index d427ea6..66808ca 100644
--- a/libs/vr/libvrflinger/display_surface.cpp
+++ b/libs/vr/libvrflinger/display_surface.cpp
@@ -26,7 +26,7 @@
     : SurfaceChannel(service, surface_id, SurfaceTypeEnum::Normal,
                      sizeof(DisplaySurfaceMetadata)),
       process_id_(process_id),
-      posted_buffers_(kMaxPostedBuffers),
+      acquired_buffers_(kMaxPostedBuffers),
       video_mesh_surfaces_updated_(false),
       width_(width),
       height_(height),
@@ -40,7 +40,6 @@
       manager_visible_(false),
       manager_z_order_(0),
       manager_blur_(0.0f),
-      allocated_buffer_index_(0),
       layer_order_(0) {}
 
 DisplaySurface::~DisplaySurface() {
@@ -84,71 +83,107 @@
   client_blur_behind_ = blur_behind;
 }
 
+void DisplaySurface::DequeueBuffersLocked() {
+  if (consumer_queue_ == nullptr) {
+    ALOGE(
+        "DisplaySurface::DequeueBuffersLocked: Consumer queue is not "
+        "initialized.");
+    return;
+  }
+
+  size_t slot;
+  uint64_t sequence;
+  while (true) {
+    LocalHandle acquire_fence;
+    auto buffer_consumer =
+        consumer_queue_->Dequeue(0, &slot, &sequence, &acquire_fence);
+    if (!buffer_consumer) {
+      ALOGD_IF(TRACE,
+               "DisplaySurface::DequeueBuffersLocked: We have dequeued all "
+               "available buffers.");
+      return;
+    }
+
+    if (!IsVisible()) {
+      ATRACE_NAME("DropFrameOnInvisibleSurface");
+      ALOGD_IF(TRACE,
+               "DisplaySurface::DequeueBuffersLocked: Discarding buffer_id=%d "
+               "on invisible surface.",
+               buffer_consumer->id());
+      buffer_consumer->Discard();
+      continue;
+    }
+
+    if (acquired_buffers_.IsFull()) {
+      ALOGE(
+          "DisplaySurface::DequeueBuffersLocked: Posted buffers full, "
+          "overwriting.");
+      acquired_buffers_.PopBack();
+    }
+
+    acquired_buffers_.Append(
+        AcquiredBuffer(buffer_consumer, std::move(acquire_fence), sequence));
+  }
+}
+
+AcquiredBuffer DisplaySurface::AcquireCurrentBuffer() {
+  std::lock_guard<std::mutex> autolock(lock_);
+  DequeueBuffersLocked();
+
+  if (acquired_buffers_.IsEmpty()) {
+    ALOGE(
+        "DisplaySurface::AcquireCurrentBuffer: attempt to acquire buffer when "
+        "none are posted.");
+    return AcquiredBuffer();
+  }
+  AcquiredBuffer buffer = std::move(acquired_buffers_.Front());
+  acquired_buffers_.PopFront();
+  ALOGD_IF(TRACE, "DisplaySurface::AcquireCurrentBuffer: buffer: %p",
+           buffer.buffer().get());
+  return buffer;
+}
+
 AcquiredBuffer DisplaySurface::AcquireNewestAvailableBuffer(
     AcquiredBuffer* skipped_buffer) {
   std::lock_guard<std::mutex> autolock(lock_);
+  DequeueBuffersLocked();
+
   AcquiredBuffer buffer;
   int frames = 0;
   // Basic latency stopgap for when the application misses a frame:
   // If the application recovers on the 2nd or 3rd (etc) frame after
   // missing, this code will skip frames to catch up by checking if
   // the next frame is also available.
-  while (!posted_buffers_.IsEmpty() && posted_buffers_.Front().IsAvailable()) {
+  while (!acquired_buffers_.IsEmpty() &&
+         acquired_buffers_.Front().IsAvailable()) {
     // Capture the skipped buffer into the result parameter.
     // Note that this API only supports skipping one buffer per vsync.
     if (frames > 0 && skipped_buffer)
       *skipped_buffer = std::move(buffer);
     ++frames;
-    buffer = std::move(posted_buffers_.Front());
-    posted_buffers_.PopFront();
+    buffer = std::move(acquired_buffers_.Front());
+    acquired_buffers_.PopFront();
     if (frames == 2)
       break;
   }
+  ALOGD_IF(TRACE, "DisplaySurface::AcquireNewestAvailableBuffer: buffer: %p",
+           buffer.buffer().get());
   return buffer;
 }
 
-bool DisplaySurface::IsBufferAvailable() const {
+bool DisplaySurface::IsBufferAvailable() {
   std::lock_guard<std::mutex> autolock(lock_);
-  return !posted_buffers_.IsEmpty() && posted_buffers_.Front().IsAvailable();
+  DequeueBuffersLocked();
+
+  return !acquired_buffers_.IsEmpty() &&
+         acquired_buffers_.Front().IsAvailable();
 }
 
-bool DisplaySurface::IsBufferPosted() const {
+bool DisplaySurface::IsBufferPosted() {
   std::lock_guard<std::mutex> autolock(lock_);
-  return !posted_buffers_.IsEmpty();
-}
+  DequeueBuffersLocked();
 
-AcquiredBuffer DisplaySurface::AcquireCurrentBuffer() {
-  std::lock_guard<std::mutex> autolock(lock_);
-  if (posted_buffers_.IsEmpty()) {
-    ALOGE("Error: attempt to acquire buffer when none are posted.");
-    return AcquiredBuffer();
-  }
-  AcquiredBuffer buffer = std::move(posted_buffers_.Front());
-  posted_buffers_.PopFront();
-  return buffer;
-}
-
-int DisplaySurface::GetConsumers(std::vector<LocalChannelHandle>* consumers) {
-  std::lock_guard<std::mutex> autolock(lock_);
-  std::vector<LocalChannelHandle> items;
-
-  for (auto pair : buffers_) {
-    const auto& buffer = pair.second;
-
-    Status<LocalChannelHandle> consumer_channel = buffer->CreateConsumer();
-    if (!consumer_channel) {
-      ALOGE(
-          "DisplaySurface::GetConsumers: Failed to get a new consumer for "
-          "buffer %d: %s",
-          buffer->id(), consumer_channel.GetErrorMessage().c_str());
-      return -consumer_channel.error();
-    }
-
-    items.push_back(consumer_channel.take());
-  }
-
-  *consumers = std::move(items);
-  return 0;
+  return !acquired_buffers_.IsEmpty();
 }
 
 int DisplaySurface::HandleMessage(pdx::Message& message) {
@@ -158,9 +193,9 @@
           *this, &DisplaySurface::OnClientSetAttributes, message);
       break;
 
-    case DisplayRPC::AllocateBuffer::Opcode:
-      DispatchRemoteMethod<DisplayRPC::AllocateBuffer>(
-          *this, &DisplaySurface::OnAllocateBuffer, message);
+    case DisplayRPC::CreateBufferQueue::Opcode:
+      DispatchRemoteMethod<DisplayRPC::CreateBufferQueue>(
+          *this, &DisplaySurface::OnCreateBufferQueue, message);
       break;
 
     case DisplayRPC::CreateVideoMeshSurface::Opcode:
@@ -226,58 +261,20 @@
   return 0;
 }
 
-// Allocates a new buffer for the DisplaySurface associated with this channel.
-std::pair<uint32_t, LocalChannelHandle> DisplaySurface::OnAllocateBuffer(
-    pdx::Message& message) {
-  // Inject flag to enable framebuffer compression for the application buffers.
-  // TODO(eieio,jbates): Make this configurable per hardware platform.
-  const int usage = usage_ | GRALLOC_USAGE_QCOM_FRAMEBUFFER_COMPRESSION;
-  const int slice_count =
-      (flags_ & static_cast<int>(DisplaySurfaceFlagsEnum::SeparateGeometry))
-          ? 2
-          : 1;
+LocalChannelHandle DisplaySurface::OnCreateBufferQueue(Message& message) {
+  ATRACE_NAME("DisplaySurface::OnCreateBufferQueue");
 
-  ALOGI_IF(
-      TRACE,
-      "DisplaySurface::OnAllocateBuffer: width=%d height=%d format=%x usage=%x "
-      "slice_count=%d",
-      width_, height_, format_, usage, slice_count);
-
-  // Create a producer buffer to hand back to the sender.
-  auto producer = BufferProducer::Create(width_, height_, format_, usage,
-                                         sizeof(uint64_t), slice_count);
-  if (!producer)
-    REPLY_ERROR_RETURN(message, EINVAL, {});
-
-  // Create and import a consumer attached to the producer.
-  Status<LocalChannelHandle> consumer_channel = producer->CreateConsumer();
-  if (!consumer_channel)
-    REPLY_ERROR_RETURN(message, consumer_channel.error(), {});
-
-  std::shared_ptr<BufferConsumer> consumer =
-      BufferConsumer::Import(consumer_channel.take());
-  if (!consumer)
-    REPLY_ERROR_RETURN(message, ENOMEM, {});
-
-  // Add the consumer to this surface.
-  int err = AddConsumer(consumer);
-  if (err < 0) {
-    ALOGE("DisplaySurface::OnAllocateBuffer: failed to add consumer: buffer=%d",
-          consumer->id());
-    REPLY_ERROR_RETURN(message, -err, {});
+  if (consumer_queue_ != nullptr) {
+    ALOGE(
+        "DisplaySurface::OnCreateBufferQueue: A ProdcuerQueue has already been "
+        "created and transported to DisplayClient.");
+    REPLY_ERROR_RETURN(message, EALREADY, {});
   }
 
-  // Move the channel handle so that it doesn't get closed when the producer
-  // goes out of scope.
-  std::pair<uint32_t, LocalChannelHandle> return_value(
-      allocated_buffer_index_, std::move(producer->GetChannelHandle()));
+  auto producer = ProducerQueue::Create<uint64_t>();
+  consumer_queue_ = producer->CreateConsumerQueue();
 
-  // Save buffer index, associated with the buffer id so that it can be looked
-  // up later.
-  buffer_id_to_index_[consumer->id()] = allocated_buffer_index_;
-  ++allocated_buffer_index_;
-
-  return return_value;
+  return std::move(producer->GetChannelHandle());
 }
 
 RemoteChannelHandle DisplaySurface::OnCreateVideoMeshSurface(
@@ -319,103 +316,6 @@
   return status.take();
 }
 
-int DisplaySurface::AddConsumer(
-    const std::shared_ptr<BufferConsumer>& consumer) {
-  ALOGD_IF(TRACE, "DisplaySurface::AddConsumer: buffer_id=%d", consumer->id());
-  // Add the consumer to the epoll dispatcher, edge-triggered.
-  int err = service()->dispatcher_.AddEventHandler(
-      consumer->event_fd(), EPOLLET | EPOLLIN | EPOLLHUP,
-      std::bind(&DisplaySurface::HandleConsumerEvents,
-                std::static_pointer_cast<DisplaySurface>(shared_from_this()),
-                consumer, std::placeholders::_1));
-  if (err) {
-    ALOGE(
-        "DisplaySurface::AddConsumer: failed to add epoll event handler for "
-        "consumer: %s",
-        strerror(-err));
-    return err;
-  }
-
-  // Add the consumer to the list of buffers for this surface.
-  std::lock_guard<std::mutex> autolock(lock_);
-  buffers_.insert(std::make_pair(consumer->id(), consumer));
-  return 0;
-}
-
-void DisplaySurface::RemoveConsumer(
-    const std::shared_ptr<BufferConsumer>& consumer) {
-  ALOGD_IF(TRACE, "DisplaySurface::RemoveConsumer: buffer_id=%d",
-           consumer->id());
-  service()->dispatcher_.RemoveEventHandler(consumer->event_fd());
-
-  std::lock_guard<std::mutex> autolock(lock_);
-  buffers_.erase(consumer->id());
-}
-
-void DisplaySurface::RemoveConsumerUnlocked(
-    const std::shared_ptr<BufferConsumer>& consumer) {
-  ALOGD_IF(TRACE, "DisplaySurface::RemoveConsumerUnlocked: buffer_id=%d",
-           consumer->id());
-  service()->dispatcher_.RemoveEventHandler(consumer->event_fd());
-  buffers_.erase(consumer->id());
-}
-
-void DisplaySurface::OnPostConsumer(
-    const std::shared_ptr<BufferConsumer>& consumer) {
-  ATRACE_NAME("DisplaySurface::OnPostConsumer");
-  std::lock_guard<std::mutex> autolock(lock_);
-
-  if (!IsVisible()) {
-    ALOGD_IF(TRACE,
-             "DisplaySurface::OnPostConsumer: Discarding buffer_id=%d on "
-             "invisible surface.",
-             consumer->id());
-    consumer->Discard();
-    return;
-  }
-
-  if (posted_buffers_.IsFull()) {
-    ALOGE("Error: posted buffers full, overwriting");
-    posted_buffers_.PopBack();
-  }
-
-  int error;
-  posted_buffers_.Append(AcquiredBuffer(consumer, &error));
-
-  // Remove the consumer if the other end was closed.
-  if (posted_buffers_.Back().IsEmpty() && error == -EPIPE)
-    RemoveConsumerUnlocked(consumer);
-}
-
-void DisplaySurface::HandleConsumerEvents(
-    const std::shared_ptr<BufferConsumer>& consumer, int events) {
-  auto status = consumer->GetEventMask(events);
-  if (!status) {
-    ALOGW(
-        "DisplaySurface::HandleConsumerEvents: Failed to get event mask for "
-        "consumer: %s",
-        status.GetErrorMessage().c_str());
-    return;
-  }
-
-  events = status.get();
-  if (events & EPOLLHUP) {
-    ALOGD_IF(TRACE,
-             "DisplaySurface::HandleConsumerEvents: removing event handler for "
-             "buffer=%d",
-             consumer->id());
-    RemoveConsumer(consumer);
-  } else if (events & EPOLLIN) {
-    // BufferHub uses EPOLLIN to signal consumer ownership.
-    ALOGD_IF(TRACE,
-             "DisplaySurface::HandleConsumerEvents: posting buffer=%d for "
-             "process=%d",
-             consumer->id(), process_id_);
-
-    OnPostConsumer(consumer);
-  }
-}
-
 std::vector<std::shared_ptr<VideoMeshSurface>>
 DisplaySurface::GetVideoMeshSurfaces() {
   std::lock_guard<std::mutex> autolock(lock_);
diff --git a/libs/vr/libvrflinger/display_surface.h b/libs/vr/libvrflinger/display_surface.h
index fa34057..feb173e 100644
--- a/libs/vr/libvrflinger/display_surface.h
+++ b/libs/vr/libvrflinger/display_surface.h
@@ -13,7 +13,6 @@
 #include <vector>
 
 #include "acquired_buffer.h"
-#include "epoll_event_dispatcher.h"
 #include "surface_channel.h"
 #include "video_mesh_surface.h"
 
@@ -65,19 +64,8 @@
     return buffer_id_to_index_[buffer_id];
   }
 
-  // Gets a new set of consumers for all of the surface's buffers. These
-  // consumers are independent from the consumers maintained internally to the
-  // surface and may be passed to other processes over IPC.
-  int GetConsumers(std::vector<pdx::LocalChannelHandle>* consumers);
-
-  template <class A>
-  void ForEachBuffer(A action) {
-    std::lock_guard<std::mutex> autolock(lock_);
-    std::for_each(buffers_.begin(), buffers_.end(), action);
-  }
-
-  bool IsBufferAvailable() const;
-  bool IsBufferPosted() const;
+  bool IsBufferAvailable();
+  bool IsBufferPosted();
   AcquiredBuffer AcquireCurrentBuffer();
 
   // Get the newest buffer. Up to one buffer will be skipped. If a buffer is
@@ -119,12 +107,6 @@
   // Returns whether a frame is available without locking the mutex.
   bool IsFrameAvailableNoLock() const;
 
-  // Handles epoll events for BufferHub consumers. Events are mainly generated
-  // by producers posting buffers ready for display. This handler runs on the
-  // epoll event thread.
-  void HandleConsumerEvents(const std::shared_ptr<BufferConsumer>& consumer,
-                            int events);
-
   // Dispatches display surface messages to the appropriate handlers. This
   // handler runs on the displayd message dispatch thread.
   int HandleMessage(pdx::Message& message) override;
@@ -133,33 +115,22 @@
   int OnClientSetAttributes(pdx::Message& message,
                             const DisplaySurfaceAttributes& attributes);
 
-  // Allocates a buffer with the display surface geometry and settings and
-  // returns it to the client.
-  std::pair<uint32_t, pdx::LocalChannelHandle> OnAllocateBuffer(
-      pdx::Message& message);
+  // Creates a BufferHubQueue associated with this surface and returns the PDX
+  // handle of its producer side to the client.
+  pdx::LocalChannelHandle OnCreateBufferQueue(pdx::Message& message);
 
-  // Creates a video mesh surface associated with this surface and returns it
-  // to the client.
+  // Creates a video mesh surface associated with this surface and returns its
+  // PDX handle to the client.
   pdx::RemoteChannelHandle OnCreateVideoMeshSurface(pdx::Message& message);
 
-  // Sets the current buffer for the display surface, discarding the previous
-  // buffer if it is not already claimed. Runs on the epoll event thread.
-  void OnPostConsumer(const std::shared_ptr<BufferConsumer>& consumer);
-
   // Client interface (called through IPC) to set visibility and z order.
   void ClientSetVisible(bool visible);
   void ClientSetZOrder(int z_order);
   void ClientSetExcludeFromBlur(bool exclude_from_blur);
   void ClientSetBlurBehind(bool blur_behind);
 
-  // Runs on the displayd message dispatch thread.
-  int AddConsumer(const std::shared_ptr<BufferConsumer>& consumer);
-
-  // Runs on the epoll event thread.
-  void RemoveConsumer(const std::shared_ptr<BufferConsumer>& consumer);
-
-  // Runs on the epoll and display post thread.
-  void RemoveConsumerUnlocked(const std::shared_ptr<BufferConsumer>& consumer);
+  // Dequeue all available buffers from the consumer queue.
+  void DequeueBuffersLocked();
 
   DisplaySurface(const DisplaySurface&) = delete;
   void operator=(const DisplaySurface&) = delete;
@@ -169,11 +140,16 @@
   // Synchronizes access to mutable state below between message dispatch thread,
   // epoll event thread, and frame post thread.
   mutable std::mutex lock_;
-  std::unordered_map<int, std::shared_ptr<BufferConsumer>> buffers_;
+
+  // The consumer end of a BufferHubQueue. VrFlinger allocates and controls the
+  // buffer queue and pass producer end to the app and the consumer end to
+  // compositor.
+  // TODO(jwcai) Add support for multiple buffer queues per display surface.
+  std::shared_ptr<ConsumerQueue> consumer_queue_;
 
   // In a triple-buffered surface, up to kMaxPostedBuffers buffers may be
   // posted and pending.
-  RingBuffer<AcquiredBuffer> posted_buffers_;
+  RingBuffer<AcquiredBuffer> acquired_buffers_;
 
   // Provides access to VideoMeshSurface. Here we don't want to increase
   // the reference count immediately on allocation, will leave it into
@@ -194,8 +170,6 @@
   bool manager_visible_;
   int manager_z_order_;
   float manager_blur_;
-  // The monotonically increasing index for allocated buffers in this surface.
-  uint32_t allocated_buffer_index_;
   int layer_order_;
 
   // Maps from the buffer id to the corresponding allocated buffer index.
diff --git a/libs/vr/libvrflinger/epoll_event_dispatcher.cpp b/libs/vr/libvrflinger/epoll_event_dispatcher.cpp
deleted file mode 100644
index b37e76e..0000000
--- a/libs/vr/libvrflinger/epoll_event_dispatcher.cpp
+++ /dev/null
@@ -1,142 +0,0 @@
-#include "epoll_event_dispatcher.h"
-
-#include <log/log.h>
-#include <sys/epoll.h>
-#include <sys/eventfd.h>
-#include <sys/prctl.h>
-
-#include <dvr/performance_client_api.h>
-
-namespace android {
-namespace dvr {
-
-EpollEventDispatcher::EpollEventDispatcher()
-    : exit_thread_(false), epoll_fd_(-1), event_fd_(-1) {
-  epoll_fd_ = epoll_create(64);
-  if (epoll_fd_ < 0) {
-    ALOGE("Failed to create epoll fd: %s", strerror(errno));
-    return;
-  }
-
-  event_fd_ = eventfd(0, EFD_CLOEXEC | EFD_NONBLOCK);
-  if (event_fd_ < 0) {
-    ALOGE("Failed to create event for epolling: %s", strerror(errno));
-    return;
-  }
-
-  // Add watch for eventfd. This should only watch for EPOLLIN, which gets set
-  // when eventfd_write occurs. Use "this" as a unique sentinal value to
-  // identify events from the event fd.
-  epoll_event event = {.events = EPOLLIN, .data = {.ptr = this}};
-  if (epoll_ctl(epoll_fd_, EPOLL_CTL_ADD, event_fd_, &event) < 0) {
-    ALOGE("Failed to add eventfd to epoll set because: %s", strerror(errno));
-    return;
-  }
-
-  thread_ = std::thread(&EpollEventDispatcher::EventThread, this);
-}
-
-EpollEventDispatcher::~EpollEventDispatcher() {
-  Stop();
-
-  close(epoll_fd_);
-  close(event_fd_);
-}
-
-void EpollEventDispatcher::Stop() {
-  exit_thread_.store(true);
-  eventfd_write(event_fd_, 1);
-}
-
-int EpollEventDispatcher::AddEventHandler(int fd, int event_mask,
-                                          Handler handler) {
-  std::lock_guard<std::mutex> lock(lock_);
-
-  epoll_event event;
-  event.events = event_mask;
-  event.data.ptr = &(handlers_[fd] = handler);
-
-  ALOGD_IF(
-      TRACE,
-      "EpollEventDispatcher::AddEventHandler: fd=%d event_mask=0x%x handler=%p",
-      fd, event_mask, event.data.ptr);
-
-  int err = epoll_ctl(epoll_fd_, EPOLL_CTL_ADD, fd, &event);
-  return err < 0 ? -errno : 0;
-}
-
-int EpollEventDispatcher::RemoveEventHandler(int fd) {
-  ALOGD_IF(TRACE, "EpollEventDispatcher::RemoveEventHandler: fd=%d", fd);
-  std::lock_guard<std::mutex> lock(lock_);
-
-  epoll_event dummy;  // See BUGS in man 2 epoll_ctl.
-  if (epoll_ctl(epoll_fd_, EPOLL_CTL_DEL, fd, &dummy) < 0) {
-    ALOGE("Failed to remove fd from epoll set because: %s", strerror(errno));
-    return -errno;
-  }
-
-  // If the fd was valid above, add it to the list of ids to remove.
-  removed_handlers_.push_back(fd);
-
-  // Wake up the event thread to clean up.
-  eventfd_write(event_fd_, 1);
-
-  return 0;
-}
-
-void EpollEventDispatcher::EventThread() {
-  prctl(PR_SET_NAME, reinterpret_cast<unsigned long>("EpollEvent"), 0, 0, 0);
-
-  const int error = dvrSetSchedulerClass(0, "graphics");
-  LOG_ALWAYS_FATAL_IF(
-      error < 0,
-      "EpollEventDispatcher::EventThread: Failed to set scheduler class: %s",
-      strerror(-error));
-
-  const size_t kMaxNumEvents = 128;
-  epoll_event events[kMaxNumEvents];
-
-  while (!exit_thread_.load()) {
-    int num_events = epoll_wait(epoll_fd_, events, kMaxNumEvents, -1);
-    if (num_events < 0 && errno != EINTR)
-      break;
-
-    ALOGD_IF(TRACE, "EpollEventDispatcher::EventThread: num_events=%d",
-             num_events);
-
-    for (int i = 0; i < num_events; i++) {
-      ALOGD_IF(
-          TRACE,
-          "EpollEventDispatcher::EventThread: event %d: handler=%p events=0x%x",
-          i, events[i].data.ptr, events[i].events);
-
-      if (events[i].data.ptr == this) {
-        // Clear pending event on event_fd_. Serialize the read with respect to
-        // writes from other threads.
-        std::lock_guard<std::mutex> lock(lock_);
-        eventfd_t value;
-        eventfd_read(event_fd_, &value);
-      } else {
-        auto handler = reinterpret_cast<Handler*>(events[i].data.ptr);
-        if (handler)
-          (*handler)(events[i].events);
-      }
-    }
-
-    // Remove any handlers that have been posted for removal. This is done here
-    // instead of in RemoveEventHandler() to prevent races between the dispatch
-    // thread and the code requesting the removal. Handlers are guaranteed to
-    // stay alive between exiting epoll_wait() and the dispatch loop above.
-    std::lock_guard<std::mutex> lock(lock_);
-    for (auto handler_fd : removed_handlers_) {
-      ALOGD_IF(TRACE,
-               "EpollEventDispatcher::EventThread: removing handler: fd=%d",
-               handler_fd);
-      handlers_.erase(handler_fd);
-    }
-    removed_handlers_.clear();
-  }
-}
-
-}  // namespace dvr
-}  // namespace android
diff --git a/libs/vr/libvrflinger/epoll_event_dispatcher.h b/libs/vr/libvrflinger/epoll_event_dispatcher.h
deleted file mode 100644
index 43bca2e..0000000
--- a/libs/vr/libvrflinger/epoll_event_dispatcher.h
+++ /dev/null
@@ -1,61 +0,0 @@
-#ifndef ANDROID_DVR_SERVICES_DISPLAYD_EPOLL_EVENT_DISPATCHER_H_
-#define ANDROID_DVR_SERVICES_DISPLAYD_EPOLL_EVENT_DISPATCHER_H_
-
-#include <sys/epoll.h>
-
-#include <atomic>
-#include <functional>
-#include <mutex>
-#include <thread>
-#include <unordered_map>
-#include <vector>
-
-namespace android {
-namespace dvr {
-
-class EpollEventDispatcher {
- public:
-  // Function type for event handlers. The handler receives a bitmask of the
-  // epoll events that occurred on the file descriptor associated with the
-  // handler.
-  using Handler = std::function<void(int)>;
-
-  EpollEventDispatcher();
-  ~EpollEventDispatcher();
-
-  // |handler| is called on the internal dispatch thread when |fd| is signaled
-  // by events in |event_mask|.
-  // Return 0 on success or a negative error code on failure.
-  int AddEventHandler(int fd, int event_mask, Handler handler);
-  int RemoveEventHandler(int fd);
-
-  void Stop();
-
- private:
-  void EventThread();
-
-  std::thread thread_;
-  std::atomic<bool> exit_thread_;
-
-  // Protects handlers_ and removed_handlers_ and serializes operations on
-  // epoll_fd_ and event_fd_.
-  std::mutex lock_;
-
-  // Maintains a map of fds to event handlers. This is primarily to keep any
-  // references alive that may be bound in the std::function instances. It is
-  // not used at dispatch time to avoid performance problems with different
-  // versions of std::unordered_map.
-  std::unordered_map<int, Handler> handlers_;
-
-  // List of fds to be removed from the map. The actual removal is performed
-  // by the event dispatch thread to avoid races.
-  std::vector<int> removed_handlers_;
-
-  int epoll_fd_;
-  int event_fd_;
-};
-
-}  // namespace dvr
-}  // namespace android
-
-#endif  // ANDROID_DVR_SERVICES_DISPLAYD_EPOLL_EVENT_DISPATCHER_H_
diff --git a/libs/vr/libvrflinger/hardware_composer.cpp b/libs/vr/libvrflinger/hardware_composer.cpp
index f801d9b..53c2ac2 100644
--- a/libs/vr/libvrflinger/hardware_composer.cpp
+++ b/libs/vr/libvrflinger/hardware_composer.cpp
@@ -1485,6 +1485,8 @@
     handle = acquired_buffer_.buffer()->native_handle();
     acquire_fence_fd_.Reset(acquired_buffer_.ClaimAcquireFence().Release());
   } else {
+    // TODO(jwcai) Note: this is the GPU compositor's layer, and we need the
+    // mechanism to accept distorted layers from VrCore.
     right = direct_buffer_->width();
     bottom = direct_buffer_->height();
     handle = direct_buffer_->handle();
diff --git a/libs/vr/libvrsensor/Android.bp b/libs/vr/libvrsensor/Android.bp
index 376630e..f201ea6 100644
--- a/libs/vr/libvrsensor/Android.bp
+++ b/libs/vr/libvrsensor/Android.bp
@@ -23,6 +23,7 @@
 
 staticLibraries = [
     "libbufferhub",
+    "libbufferhubqueue",
     "libdvrcommon",
     "libpdx_default_transport",
 ]